Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
taosdata
TDengine
提交
f924c1b8
T
TDengine
项目概览
taosdata
/
TDengine
1 年多 前同步成功
通知
1185
Star
22016
Fork
4786
代码
文件
提交
分支
Tags
贡献者
分支图
Diff
Issue
1
列表
看板
标记
里程碑
合并请求
0
Wiki
0
Wiki
分析
仓库
DevOps
项目成员
Pages
T
TDengine
项目概览
项目概览
详情
发布
仓库
仓库
文件
提交
分支
标签
贡献者
分支图
比较
Issue
1
Issue
1
列表
看板
标记
里程碑
合并请求
0
合并请求
0
Pages
分析
分析
仓库分析
DevOps
Wiki
0
Wiki
成员
成员
收起侧边栏
关闭侧边栏
动态
分支图
创建新Issue
提交
Issue看板
提交
f924c1b8
编写于
3月 14, 2022
作者:
M
Minghao Li
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
sync index
上级
7c868aea
变更
12
隐藏空白更改
内联
并排
Showing
12 changed file
with
155 addition
and
17 deletion
+155
-17
source/libs/sync/inc/syncInt.h
source/libs/sync/inc/syncInt.h
+1
-0
source/libs/sync/inc/syncMessage.h
source/libs/sync/inc/syncMessage.h
+2
-1
source/libs/sync/src/syncAppendEntries.c
source/libs/sync/src/syncAppendEntries.c
+6
-1
source/libs/sync/src/syncAppendEntriesReply.c
source/libs/sync/src/syncAppendEntriesReply.c
+44
-1
source/libs/sync/src/syncElection.c
source/libs/sync/src/syncElection.c
+1
-1
source/libs/sync/src/syncMain.c
source/libs/sync/src/syncMain.c
+2
-0
source/libs/sync/src/syncMessage.c
source/libs/sync/src/syncMessage.c
+5
-3
source/libs/sync/src/syncReplication.c
source/libs/sync/src/syncReplication.c
+23
-6
source/libs/sync/src/syncRequestVote.c
source/libs/sync/src/syncRequestVote.c
+35
-1
source/libs/sync/src/syncRequestVoteReply.c
source/libs/sync/src/syncRequestVoteReply.c
+34
-1
source/libs/sync/test/syncRequestVoteTest.cpp
source/libs/sync/test/syncRequestVoteTest.cpp
+1
-1
source/libs/sync/test/syncRpcMsgTest.cpp
source/libs/sync/test/syncRpcMsgTest.cpp
+1
-1
未找到文件。
source/libs/sync/inc/syncInt.h
浏览文件 @
f924c1b8
...
@@ -232,6 +232,7 @@ void syncNodeCandidate2Follower(SSyncNode* pSyncNode);
...
@@ -232,6 +232,7 @@ void syncNodeCandidate2Follower(SSyncNode* pSyncNode);
// raft vote --------------
// raft vote --------------
void
syncNodeVoteForTerm
(
SSyncNode
*
pSyncNode
,
SyncTerm
term
,
SRaftId
*
pRaftId
);
void
syncNodeVoteForTerm
(
SSyncNode
*
pSyncNode
,
SyncTerm
term
,
SRaftId
*
pRaftId
);
void
syncNodeVoteForSelf
(
SSyncNode
*
pSyncNode
);
void
syncNodeVoteForSelf
(
SSyncNode
*
pSyncNode
);
void
syncNodeMaybeAdvanceCommitIndex
(
SSyncNode
*
pSyncNode
);
// for debug --------------
// for debug --------------
void
syncNodePrint
(
SSyncNode
*
pObj
);
void
syncNodePrint
(
SSyncNode
*
pObj
);
...
...
source/libs/sync/inc/syncMessage.h
浏览文件 @
f924c1b8
...
@@ -196,7 +196,7 @@ typedef struct SyncRequestVote {
...
@@ -196,7 +196,7 @@ typedef struct SyncRequestVote {
SRaftId
srcId
;
SRaftId
srcId
;
SRaftId
destId
;
SRaftId
destId
;
// private data
// private data
SyncTerm
currentT
erm
;
SyncTerm
t
erm
;
SyncIndex
lastLogIndex
;
SyncIndex
lastLogIndex
;
SyncTerm
lastLogTerm
;
SyncTerm
lastLogTerm
;
}
SyncRequestVote
;
}
SyncRequestVote
;
...
@@ -287,6 +287,7 @@ typedef struct SyncAppendEntriesReply {
...
@@ -287,6 +287,7 @@ typedef struct SyncAppendEntriesReply {
SRaftId
srcId
;
SRaftId
srcId
;
SRaftId
destId
;
SRaftId
destId
;
// private data
// private data
SyncTerm
term
;
bool
success
;
bool
success
;
SyncIndex
matchIndex
;
SyncIndex
matchIndex
;
}
SyncAppendEntriesReply
;
}
SyncAppendEntriesReply
;
...
...
source/libs/sync/src/syncAppendEntries.c
浏览文件 @
f924c1b8
...
@@ -80,4 +80,9 @@
...
@@ -80,4 +80,9 @@
// /\ UNCHANGED <<serverVars, commitIndex, messages>>
// /\ UNCHANGED <<serverVars, commitIndex, messages>>
// /\ UNCHANGED <<candidateVars, leaderVars>>
// /\ UNCHANGED <<candidateVars, leaderVars>>
//
//
int32_t
syncNodeOnAppendEntriesCb
(
SSyncNode
*
ths
,
SyncAppendEntries
*
pMsg
)
{}
int32_t
syncNodeOnAppendEntriesCb
(
SSyncNode
*
ths
,
SyncAppendEntries
*
pMsg
)
{
int32_t
ret
=
0
;
syncAppendEntriesLog2
(
"==syncNodeOnAppendEntriesCb=="
,
pMsg
);
return
ret
;
}
source/libs/sync/src/syncAppendEntriesReply.c
浏览文件 @
f924c1b8
...
@@ -14,6 +14,12 @@
...
@@ -14,6 +14,12 @@
*/
*/
#include "syncAppendEntriesReply.h"
#include "syncAppendEntriesReply.h"
#include "syncIndexMgr.h"
#include "syncInt.h"
#include "syncRaftLog.h"
#include "syncRaftStore.h"
#include "syncUtil.h"
#include "syncVoteMgr.h"
// TLA+ Spec
// TLA+ Spec
// HandleAppendEntriesResponse(i, j, m) ==
// HandleAppendEntriesResponse(i, j, m) ==
...
@@ -28,4 +34,41 @@
...
@@ -28,4 +34,41 @@
// /\ Discard(m)
// /\ Discard(m)
// /\ UNCHANGED <<serverVars, candidateVars, logVars, elections>>
// /\ UNCHANGED <<serverVars, candidateVars, logVars, elections>>
//
//
int32_t
syncNodeOnAppendEntriesReplyCb
(
SSyncNode
*
ths
,
SyncAppendEntriesReply
*
pMsg
)
{}
int32_t
syncNodeOnAppendEntriesReplyCb
(
SSyncNode
*
ths
,
SyncAppendEntriesReply
*
pMsg
)
{
int32_t
ret
=
0
;
syncAppendEntriesReplyLog2
(
"==syncNodeOnAppendEntriesReplyCb=="
,
pMsg
);
if
(
pMsg
->
term
<
ths
->
pRaftStore
->
currentTerm
)
{
sTrace
(
"DropStaleResponse, receive term:%lu, current term:%lu"
,
pMsg
->
term
,
ths
->
pRaftStore
->
currentTerm
);
return
ret
;
}
// no need this code, because if I receive reply.term, then I must have sent for that term.
// if (pMsg->term > ths->pRaftStore->currentTerm) {
// syncNodeUpdateTerm(ths, pMsg->term);
// }
assert
(
pMsg
->
term
==
ths
->
pRaftStore
->
currentTerm
);
if
(
pMsg
->
success
)
{
// nextIndex = reply.matchIndex + 1
syncIndexMgrSetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
),
pMsg
->
matchIndex
+
1
);
// matchIndex = reply.matchIndex
syncIndexMgrSetIndex
(
ths
->
pMatchIndex
,
&
(
pMsg
->
srcId
),
pMsg
->
matchIndex
);
// maybe commit
syncNodeMaybeAdvanceCommitIndex
(
ths
);
}
else
{
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
));
if
(
nextIndex
>
SYNC_INDEX_BEGIN
)
{
--
nextIndex
;
}
else
{
nextIndex
=
SYNC_INDEX_BEGIN
;
}
syncIndexMgrSetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
),
nextIndex
);
}
return
ret
;
}
source/libs/sync/src/syncElection.c
浏览文件 @
f924c1b8
...
@@ -38,7 +38,7 @@ int32_t syncNodeRequestVotePeers(SSyncNode* pSyncNode) {
...
@@ -38,7 +38,7 @@ int32_t syncNodeRequestVotePeers(SSyncNode* pSyncNode) {
SyncRequestVote
*
pMsg
=
syncRequestVoteBuild
();
SyncRequestVote
*
pMsg
=
syncRequestVoteBuild
();
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
destId
=
pSyncNode
->
peersId
[
i
];
pMsg
->
destId
=
pSyncNode
->
peersId
[
i
];
pMsg
->
currentT
erm
=
pSyncNode
->
pRaftStore
->
currentTerm
;
pMsg
->
t
erm
=
pSyncNode
->
pRaftStore
->
currentTerm
;
pMsg
->
lastLogIndex
=
pSyncNode
->
pLogStore
->
getLastIndex
(
pSyncNode
->
pLogStore
);
pMsg
->
lastLogIndex
=
pSyncNode
->
pLogStore
->
getLastIndex
(
pSyncNode
->
pLogStore
);
pMsg
->
lastLogTerm
=
pSyncNode
->
pLogStore
->
getLastTerm
(
pSyncNode
->
pLogStore
);
pMsg
->
lastLogTerm
=
pSyncNode
->
pLogStore
->
getLastTerm
(
pSyncNode
->
pLogStore
);
...
...
source/libs/sync/src/syncMain.c
浏览文件 @
f924c1b8
...
@@ -587,6 +587,8 @@ void syncNodeVoteForSelf(SSyncNode* pSyncNode) {
...
@@ -587,6 +587,8 @@ void syncNodeVoteForSelf(SSyncNode* pSyncNode) {
syncRequestVoteReplyDestroy
(
pMsg
);
syncRequestVoteReplyDestroy
(
pMsg
);
}
}
void
syncNodeMaybeAdvanceCommitIndex
(
SSyncNode
*
pSyncNode
)
{}
// for debug --------------
// for debug --------------
void
syncNodePrint
(
SSyncNode
*
pObj
)
{
void
syncNodePrint
(
SSyncNode
*
pObj
)
{
char
*
serialized
=
syncNode2Str
(
pObj
);
char
*
serialized
=
syncNode2Str
(
pObj
);
...
...
source/libs/sync/src/syncMessage.c
浏览文件 @
f924c1b8
...
@@ -820,8 +820,8 @@ cJSON* syncRequestVote2Json(const SyncRequestVote* pMsg) {
...
@@ -820,8 +820,8 @@ cJSON* syncRequestVote2Json(const SyncRequestVote* pMsg) {
cJSON_AddNumberToObject
(
pDestId
,
"vgId"
,
pMsg
->
destId
.
vgId
);
cJSON_AddNumberToObject
(
pDestId
,
"vgId"
,
pMsg
->
destId
.
vgId
);
cJSON_AddItemToObject
(
pRoot
,
"destId"
,
pDestId
);
cJSON_AddItemToObject
(
pRoot
,
"destId"
,
pDestId
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
currentT
erm
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
t
erm
);
cJSON_AddStringToObject
(
pRoot
,
"
currentT
erm"
,
u64buf
);
cJSON_AddStringToObject
(
pRoot
,
"
t
erm"
,
u64buf
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
lastLogIndex
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
lastLogIndex
);
cJSON_AddStringToObject
(
pRoot
,
"lastLogIndex"
,
u64buf
);
cJSON_AddStringToObject
(
pRoot
,
"lastLogIndex"
,
u64buf
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
lastLogTerm
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
lastLogTerm
);
...
@@ -1264,9 +1264,11 @@ cJSON* syncAppendEntriesReply2Json(const SyncAppendEntriesReply* pMsg) {
...
@@ -1264,9 +1264,11 @@ cJSON* syncAppendEntriesReply2Json(const SyncAppendEntriesReply* pMsg) {
cJSON_AddNumberToObject
(
pDestId
,
"vgId"
,
pMsg
->
destId
.
vgId
);
cJSON_AddNumberToObject
(
pDestId
,
"vgId"
,
pMsg
->
destId
.
vgId
);
cJSON_AddItemToObject
(
pRoot
,
"destId"
,
pDestId
);
cJSON_AddItemToObject
(
pRoot
,
"destId"
,
pDestId
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
term
);
cJSON_AddStringToObject
(
pRoot
,
"term"
,
u64buf
);
cJSON_AddNumberToObject
(
pRoot
,
"success"
,
pMsg
->
success
);
cJSON_AddNumberToObject
(
pRoot
,
"success"
,
pMsg
->
success
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
matchIndex
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"%lu"
,
pMsg
->
matchIndex
);
cJSON_AddStringToObject
(
pRoot
,
"match
_i
ndex"
,
u64buf
);
cJSON_AddStringToObject
(
pRoot
,
"match
I
ndex"
,
u64buf
);
cJSON
*
pJson
=
cJSON_CreateObject
();
cJSON
*
pJson
=
cJSON_CreateObject
();
cJSON_AddItemToObject
(
pJson
,
"SyncAppendEntriesReply"
,
pRoot
);
cJSON_AddItemToObject
(
pJson
,
"SyncAppendEntriesReply"
,
pRoot
);
...
...
source/libs/sync/src/syncReplication.c
浏览文件 @
f924c1b8
...
@@ -17,6 +17,8 @@
...
@@ -17,6 +17,8 @@
#include "syncIndexMgr.h"
#include "syncIndexMgr.h"
#include "syncMessage.h"
#include "syncMessage.h"
#include "syncRaftEntry.h"
#include "syncRaftEntry.h"
#include "syncRaftLog.h"
#include "syncUtil.h"
// TLA+ Spec
// TLA+ Spec
// AppendEntries(i, j) ==
// AppendEntries(i, j) ==
...
@@ -50,16 +52,31 @@ int32_t syncNodeAppendEntriesPeers(SSyncNode* pSyncNode) {
...
@@ -50,16 +52,31 @@ int32_t syncNodeAppendEntriesPeers(SSyncNode* pSyncNode) {
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
SRaftId
*
pDestId
=
&
(
pSyncNode
->
peersId
[
i
]);
SRaftId
*
pDestId
=
&
(
pSyncNode
->
peersId
[
i
]);
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
pSyncNode
->
pNextIndex
,
pDestId
);
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
pSyncNode
->
pNextIndex
,
pDestId
);
SyncIndex
preLogIndex
=
nextIndex
-
1
;
SyncIndex
preLogIndex
=
nextIndex
-
1
;
SyncTerm
preLogTerm
=
0
;
if
(
preLogIndex
>=
0
)
{
SyncTerm
preLogTerm
=
0
;
if
(
preLogIndex
>=
SYNC_INDEX_BEGIN
)
{
SSyncRaftEntry
*
pPreEntry
=
pSyncNode
->
pLogStore
->
getEntry
(
pSyncNode
->
pLogStore
,
preLogIndex
);
SSyncRaftEntry
*
pPreEntry
=
pSyncNode
->
pLogStore
->
getEntry
(
pSyncNode
->
pLogStore
,
preLogIndex
);
preLogTerm
=
pPreEntry
->
term
;
preLogTerm
=
pPreEntry
->
term
;
}
else
{
preLogTerm
=
0
;
}
}
// SyncTerm lastIndex =
// pSyncNode->pLogStore->getLastIndex < nextIndex ? pSyncNode->pLogStore->getLastIndex : nextIndex;
SyncIndex
lastIndex
=
syncUtilMinIndex
(
pSyncNode
->
pLogStore
->
getLastIndex
(
pSyncNode
->
pLogStore
),
nextIndex
);
assert
(
nextIndex
==
lastIndex
);
SSyncRaftEntry
*
pEntry
=
logStoreGetEntry
(
pSyncNode
->
pLogStore
,
nextIndex
);
assert
(
pEntry
!=
NULL
);
SyncAppendEntries
*
pMsg
=
syncAppendEntriesBuild
(
pEntry
->
bytes
);
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
destId
=
*
pDestId
;
pMsg
->
prevLogIndex
=
preLogIndex
;
pMsg
->
prevLogTerm
=
preLogTerm
;
pMsg
->
commitIndex
=
pSyncNode
->
commitIndex
;
pMsg
->
dataLen
=
pEntry
->
bytes
;
// add pEntry into msg
syncNodeAppendEntries
(
pSyncNode
,
pDestId
,
pMsg
);
}
}
return
ret
;
return
ret
;
...
...
source/libs/sync/src/syncRequestVote.c
浏览文件 @
f924c1b8
...
@@ -14,6 +14,10 @@
...
@@ -14,6 +14,10 @@
*/
*/
#include "syncRequestVote.h"
#include "syncRequestVote.h"
#include "syncInt.h"
#include "syncRaftStore.h"
#include "syncUtil.h"
#include "syncVoteMgr.h"
// TLA+ Spec
// TLA+ Spec
// HandleRequestVoteRequest(i, j, m) ==
// HandleRequestVoteRequest(i, j, m) ==
...
@@ -37,4 +41,34 @@
...
@@ -37,4 +41,34 @@
// m)
// m)
// /\ UNCHANGED <<state, currentTerm, candidateVars, leaderVars, logVars>>
// /\ UNCHANGED <<state, currentTerm, candidateVars, leaderVars, logVars>>
//
//
int32_t
syncNodeOnRequestVoteCb
(
SSyncNode
*
ths
,
SyncRequestVote
*
pMsg
)
{}
int32_t
syncNodeOnRequestVoteCb
(
SSyncNode
*
ths
,
SyncRequestVote
*
pMsg
)
{
int32_t
ret
=
0
;
syncRequestVoteLog2
(
"==syncNodeOnRequestVoteCb=="
,
pMsg
);
if
(
pMsg
->
term
>
ths
->
pRaftStore
->
currentTerm
)
{
syncNodeUpdateTerm
(
ths
,
pMsg
->
term
);
}
assert
(
pMsg
->
term
<=
ths
->
pRaftStore
->
currentTerm
);
bool
logOK
=
(
pMsg
->
lastLogTerm
>
ths
->
pLogStore
->
getLastTerm
(
ths
->
pLogStore
))
||
((
pMsg
->
lastLogTerm
==
ths
->
pLogStore
->
getLastTerm
(
ths
->
pLogStore
))
&&
(
pMsg
->
lastLogIndex
>=
ths
->
pLogStore
->
getLastIndex
(
ths
->
pLogStore
)));
bool
grant
=
(
pMsg
->
term
==
ths
->
pRaftStore
->
currentTerm
)
&&
logOK
&&
((
!
raftStoreHasVoted
(
ths
->
pRaftStore
))
||
(
syncUtilSameId
(
&
(
ths
->
pRaftStore
->
voteFor
),
&
(
pMsg
->
srcId
))));
if
(
grant
)
{
raftStoreVote
(
ths
->
pRaftStore
,
&
(
pMsg
->
srcId
));
}
SyncRequestVoteReply
*
pReply
=
syncRequestVoteReplyBuild
();
pReply
->
srcId
=
ths
->
myRaftId
;
pReply
->
destId
=
pMsg
->
srcId
;
pReply
->
term
=
ths
->
pRaftStore
->
currentTerm
;
pReply
->
voteGranted
=
grant
;
SRpcMsg
rpcMsg
;
syncRequestVoteReply2RpcMsg
(
pReply
,
&
rpcMsg
);
syncNodeSendMsgById
(
&
pReply
->
destId
,
ths
,
&
rpcMsg
);
syncRequestVoteReplyDestroy
(
pReply
);
return
ret
;
}
source/libs/sync/src/syncRequestVoteReply.c
浏览文件 @
f924c1b8
...
@@ -14,6 +14,10 @@
...
@@ -14,6 +14,10 @@
*/
*/
#include "syncRequestVoteReply.h"
#include "syncRequestVoteReply.h"
#include "syncInt.h"
#include "syncRaftStore.h"
#include "syncUtil.h"
#include "syncVoteMgr.h"
// TLA+ Spec
// TLA+ Spec
// HandleRequestVoteResponse(i, j, m) ==
// HandleRequestVoteResponse(i, j, m) ==
...
@@ -32,4 +36,33 @@
...
@@ -32,4 +36,33 @@
// /\ Discard(m)
// /\ Discard(m)
// /\ UNCHANGED <<serverVars, votedFor, leaderVars, logVars>>
// /\ UNCHANGED <<serverVars, votedFor, leaderVars, logVars>>
//
//
int32_t
syncNodeOnRequestVoteReplyCb
(
SSyncNode
*
ths
,
SyncRequestVoteReply
*
pMsg
)
{}
int32_t
syncNodeOnRequestVoteReplyCb
(
SSyncNode
*
ths
,
SyncRequestVoteReply
*
pMsg
)
{
int32_t
ret
=
0
;
syncRequestVoteReplyLog2
(
"==syncNodeOnRequestVoteReplyCb=="
,
pMsg
);
if
(
pMsg
->
term
<
ths
->
pRaftStore
->
currentTerm
)
{
sTrace
(
"DropStaleResponse, receive term:%lu, current term:%lu"
,
pMsg
->
term
,
ths
->
pRaftStore
->
currentTerm
);
return
ret
;
}
// no need this code, because if I receive reply.term, then I must have sent for that term.
// if (pMsg->term > ths->pRaftStore->currentTerm) {
// syncNodeUpdateTerm(ths, pMsg->term);
// }
assert
(
pMsg
->
term
==
ths
->
pRaftStore
->
currentTerm
);
if
(
ths
->
state
==
TAOS_SYNC_STATE_CANDIDATE
)
{
votesRespondAdd
(
ths
->
pVotesRespond
,
pMsg
);
if
(
pMsg
->
voteGranted
)
{
voteGrantedVote
(
ths
->
pVotesGranted
,
pMsg
);
if
(
voteGrantedMajority
(
ths
->
pVotesGranted
))
{
if
(
ths
->
pVotesGranted
->
toLeader
)
{
syncNodeCandidate2Leader
(
ths
);
ths
->
pVotesGranted
->
toLeader
=
true
;
}
}
}
}
return
ret
;
}
source/libs/sync/test/syncRequestVoteTest.cpp
浏览文件 @
f924c1b8
...
@@ -20,7 +20,7 @@ SyncRequestVote *createMsg() {
...
@@ -20,7 +20,7 @@ SyncRequestVote *createMsg() {
pMsg
->
srcId
.
vgId
=
100
;
pMsg
->
srcId
.
vgId
=
100
;
pMsg
->
destId
.
addr
=
syncUtilAddr2U64
(
"127.0.0.1"
,
5678
);
pMsg
->
destId
.
addr
=
syncUtilAddr2U64
(
"127.0.0.1"
,
5678
);
pMsg
->
destId
.
vgId
=
100
;
pMsg
->
destId
.
vgId
=
100
;
pMsg
->
currentT
erm
=
11
;
pMsg
->
t
erm
=
11
;
pMsg
->
lastLogIndex
=
22
;
pMsg
->
lastLogIndex
=
22
;
pMsg
->
lastLogTerm
=
33
;
pMsg
->
lastLogTerm
=
33
;
return
pMsg
;
return
pMsg
;
...
...
source/libs/sync/test/syncRpcMsgTest.cpp
浏览文件 @
f924c1b8
...
@@ -57,7 +57,7 @@ SyncRequestVote *createSyncRequestVote() {
...
@@ -57,7 +57,7 @@ SyncRequestVote *createSyncRequestVote() {
pMsg
->
srcId
.
vgId
=
100
;
pMsg
->
srcId
.
vgId
=
100
;
pMsg
->
destId
.
addr
=
syncUtilAddr2U64
(
"127.0.0.1"
,
5678
);
pMsg
->
destId
.
addr
=
syncUtilAddr2U64
(
"127.0.0.1"
,
5678
);
pMsg
->
destId
.
vgId
=
100
;
pMsg
->
destId
.
vgId
=
100
;
pMsg
->
currentT
erm
=
11
;
pMsg
->
t
erm
=
11
;
pMsg
->
lastLogIndex
=
22
;
pMsg
->
lastLogIndex
=
22
;
pMsg
->
lastLogTerm
=
33
;
pMsg
->
lastLogTerm
=
33
;
return
pMsg
;
return
pMsg
;
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录