Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
taosdata
TDengine
提交
c2b348be
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看板
提交
c2b348be
编写于
7月 20, 2022
作者:
M
Minghao Li
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
refactor(sync): add trace log
上级
90e7d794
变更
5
展开全部
显示空白变更内容
内联
并排
Showing
5 changed file
with
427 addition
and
167 deletion
+427
-167
source/libs/sync/src/syncAppendEntries.c
source/libs/sync/src/syncAppendEntries.c
+233
-77
source/libs/sync/src/syncAppendEntriesReply.c
source/libs/sync/src/syncAppendEntriesReply.c
+161
-65
source/libs/sync/src/syncMain.c
source/libs/sync/src/syncMain.c
+4
-2
source/libs/sync/src/syncRaftStore.c
source/libs/sync/src/syncRaftStore.c
+11
-11
source/libs/sync/src/syncReplication.c
source/libs/sync/src/syncReplication.c
+18
-12
未找到文件。
source/libs/sync/src/syncAppendEntries.c
浏览文件 @
c2b348be
此差异已折叠。
点击以展开。
source/libs/sync/src/syncAppendEntriesReply.c
浏览文件 @
c2b348be
...
...
@@ -40,29 +40,40 @@
int32_t
syncNodeOnAppendEntriesReplyCb
(
SSyncNode
*
ths
,
SyncAppendEntriesReply
*
pMsg
)
{
int32_t
ret
=
0
;
// print log
syncAppendEntriesReplyLog2
(
"==syncNodeOnAppendEntriesReplyCb=="
,
pMsg
);
// if already drop replica, do not process
if
(
!
syncNodeInRaftGroup
(
ths
,
&
(
pMsg
->
srcId
))
&&
!
ths
->
pRaftCfg
->
isStandBy
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, maybe replica already dropped"
);
return
0
;
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, maybe replica "
"already dropped"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeErrorLog
(
ths
,
logBuf
);
}
while
(
0
);
return
-
1
;
}
// drop stale response
if
(
pMsg
->
term
<
ths
->
pRaftStore
->
currentTerm
)
{
char
logBuf
[
128
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, recv-term:%"
PRIu64
", drop stale response"
,
pMsg
->
term
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, drop stale response"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeEventLog
(
ths
,
logBuf
);
return
0
;
}
}
while
(
0
);
if
(
gRaftDetailLog
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, before"
);
return
0
;
}
syncIndexMgrLog2
(
"==syncNodeOnAppendEntriesReplyCb== before pNextIndex"
,
ths
->
pNextIndex
);
syncIndexMgrLog2
(
"==syncNodeOnAppendEntriesReplyCb== before pMatchIndex"
,
ths
->
pMatchIndex
);
// no need this code, because if I receive reply.term, then I must have sent for that term.
// if (pMsg->term > ths->pRaftStore->currentTerm) {
...
...
@@ -70,9 +81,18 @@ int32_t syncNodeOnAppendEntriesReplyCb(SSyncNode* ths, SyncAppendEntriesReply* p
// }
if
(
pMsg
->
term
>
ths
->
pRaftStore
->
currentTerm
)
{
char
logBuf
[
128
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, error term, recv-term:%"
PRIu64
,
pMsg
->
term
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, error term"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeErrorLog
(
ths
,
logBuf
);
}
while
(
0
);
return
-
1
;
}
...
...
@@ -100,13 +120,23 @@ int32_t syncNodeOnAppendEntriesReplyCb(SSyncNode* ths, SyncAppendEntriesReply* p
syncIndexMgrSetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
),
nextIndex
);
}
if
(
gRaftDetailLog
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, after"
);
}
syncIndexMgrLog2
(
"==syncNodeOnAppendEntriesReplyCb== after pNextIndex"
,
ths
->
pNextIndex
);
syncIndexMgrLog2
(
"==syncNodeOnAppendEntriesReplyCb== after pMatchIndex"
,
ths
->
pMatchIndex
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
));
SyncIndex
matchIndex
=
syncIndexMgrGetIndex
(
ths
->
pMatchIndex
,
&
(
pMsg
->
srcId
));
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, after next:"
PRId64
", "
"match:"
PRId64
""
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
,
nextIndex
,
matchIndex
);
syncNodeEventLog
(
ths
,
logBuf
);
}
while
(
0
);
return
ret
;
return
0
;
}
// only start once
...
...
@@ -147,35 +177,55 @@ static void syncNodeStartSnapshotOnce(SSyncNode* ths, SyncIndex beginIndex, Sync
int32_t
syncNodeOnAppendEntriesReplySnapshot2Cb
(
SSyncNode
*
ths
,
SyncAppendEntriesReply
*
pMsg
)
{
int32_t
ret
=
0
;
// print log
// if already drop replica, do not process
if
(
!
syncNodeInRaftGroup
(
ths
,
&
(
pMsg
->
srcId
))
&&
!
ths
->
pRaftCfg
->
isStandBy
)
{
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, term:%lu, match:%ld, success:%d"
,
pMsg
->
term
,
pMsg
->
matchIndex
,
pMsg
->
success
);
syncNodeEventLog
(
ths
,
logBuf
);
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, maybe replica "
"already dropped"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeErrorLog
(
ths
,
logBuf
);
}
while
(
0
);
// if already drop replica, do not process
if
(
!
syncNodeInRaftGroup
(
ths
,
&
(
pMsg
->
srcId
))
&&
!
ths
->
pRaftCfg
->
isStandBy
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, maybe replica already dropped"
);
return
-
1
;
}
// drop stale response
if
(
pMsg
->
term
<
ths
->
pRaftStore
->
currentTerm
)
{
char
logBuf
[
128
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, recv-term:%"
PRIu64
", drop stale response"
,
pMsg
->
term
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, drop stale response"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeEventLog
(
ths
,
logBuf
);
return
-
1
;
}
while
(
0
);
return
0
;
}
// error term
if
(
pMsg
->
term
>
ths
->
pRaftStore
->
currentTerm
)
{
char
logBuf
[
128
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, error term, recv-term:%"
PRIu64
,
pMsg
->
term
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, error term"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeErrorLog
(
ths
,
logBuf
);
}
while
(
0
);
return
-
1
;
}
...
...
@@ -293,35 +343,62 @@ int32_t syncNodeOnAppendEntriesReplySnapshot2Cb(SSyncNode* ths, SyncAppendEntrie
}
while
(
0
);
}
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
));
SyncIndex
matchIndex
=
syncIndexMgrGetIndex
(
ths
->
pMatchIndex
,
&
(
pMsg
->
srcId
));
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, after next:"
PRId64
", "
"match:"
PRId64
""
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
,
nextIndex
,
matchIndex
);
syncNodeEventLog
(
ths
,
logBuf
);
}
while
(
0
);
return
0
;
}
int32_t
syncNodeOnAppendEntriesReplySnapshotCb
(
SSyncNode
*
ths
,
SyncAppendEntriesReply
*
pMsg
)
{
int32_t
ret
=
0
;
// print log
syncAppendEntriesReplyLog2
(
"==syncNodeOnAppendEntriesReplySnapshotCb=="
,
pMsg
);
// if already drop replica, do not process
if
(
!
syncNodeInRaftGroup
(
ths
,
&
(
pMsg
->
srcId
))
&&
!
ths
->
pRaftCfg
->
isStandBy
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, maybe replica already dropped"
);
return
0
;
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, maybe replica "
"already dropped"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeErrorLog
(
ths
,
logBuf
);
}
while
(
0
);
return
-
1
;
}
// drop stale response
if
(
pMsg
->
term
<
ths
->
pRaftStore
->
currentTerm
)
{
char
logBuf
[
128
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, recv-term:%"
PRIu64
", drop stale response"
,
pMsg
->
term
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, drop stale response"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeEventLog
(
ths
,
logBuf
);
return
0
;
}
}
while
(
0
);
if
(
gRaftDetailLog
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, before"
);
return
0
;
}
syncIndexMgrLog2
(
"recv sync-append-entries-reply, before pNextIndex:"
,
ths
->
pNextIndex
);
syncIndexMgrLog2
(
"recv sync-append-entries-reply, before pMatchIndex:"
,
ths
->
pMatchIndex
);
// no need this code, because if I receive reply.term, then I must have sent for that term.
// if (pMsg->term > ths->pRaftStore->currentTerm) {
...
...
@@ -329,9 +406,18 @@ int32_t syncNodeOnAppendEntriesReplySnapshotCb(SSyncNode* ths, SyncAppendEntries
// }
if
(
pMsg
->
term
>
ths
->
pRaftStore
->
currentTerm
)
{
char
logBuf
[
128
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply, error term, recv-term:%"
PRIu64
,
pMsg
->
term
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, error term"
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
);
syncNodeErrorLog
(
ths
,
logBuf
);
}
while
(
0
);
return
-
1
;
}
...
...
@@ -404,11 +490,21 @@ int32_t syncNodeOnAppendEntriesReplySnapshotCb(SSyncNode* ths, SyncAppendEntries
}
}
if
(
gRaftDetailLog
)
{
syncNodeEventLog
(
ths
,
"recv sync-append-entries-reply, after"
);
}
syncIndexMgrLog2
(
"recv sync-append-entries-reply, after pNextIndex:"
,
ths
->
pNextIndex
);
syncIndexMgrLog2
(
"recv sync-append-entries-reply, after pMatchIndex:"
,
ths
->
pMatchIndex
);
do
{
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
pMsg
->
srcId
.
addr
,
host
,
sizeof
(
host
),
&
port
);
char
logBuf
[
256
];
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
ths
->
pNextIndex
,
&
(
pMsg
->
srcId
));
SyncIndex
matchIndex
=
syncIndexMgrGetIndex
(
ths
->
pMatchIndex
,
&
(
pMsg
->
srcId
));
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"recv sync-append-entries-reply from %s:%d {term:"
PRIu64
", pterm:"
PRIu64
", success:%d, match:"
PRId64
"}, after next:"
PRId64
", "
"match:"
PRId64
""
,
host
,
port
,
pMsg
->
term
,
pMsg
->
privateTerm
,
pMsg
->
success
,
pMsg
->
matchIndex
,
nextIndex
,
matchIndex
);
syncNodeEventLog
(
ths
,
logBuf
);
}
while
(
0
);
return
0
;
}
\ No newline at end of file
source/libs/sync/src/syncMain.c
浏览文件 @
c2b348be
...
...
@@ -1564,7 +1564,8 @@ void syncNodeEventLog(const SSyncNode* pSyncNode, char* str) {
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"%s"
,
str
);
}
// sDebug("%s", logBuf);
sInfo
(
"%s"
,
logBuf
);
// sInfo("%s", logBuf);
sTrace
(
"%s"
,
logBuf
);
}
else
{
int
len
=
256
+
userStrLen
;
...
...
@@ -1586,7 +1587,8 @@ void syncNodeEventLog(const SSyncNode* pSyncNode, char* str) {
snprintf
(
s
,
len
,
"%s"
,
str
);
}
// sDebug("%s", s);
sInfo
(
"%s"
,
s
);
// sInfo("%s", s);
sTrace
(
"%s"
,
s
);
taosMemoryFree
(
s
);
}
...
...
source/libs/sync/src/syncRaftStore.c
浏览文件 @
c2b348be
...
...
@@ -108,10 +108,10 @@ int32_t raftStoreSerialize(SRaftStore *pRaftStore, char *buf, size_t len) {
cJSON
*
pRoot
=
cJSON_CreateObject
();
char
u64Buf
[
128
]
=
{
0
};
snprintf
(
u64Buf
,
sizeof
(
u64Buf
),
"
%lu
"
,
pRaftStore
->
currentTerm
);
snprintf
(
u64Buf
,
sizeof
(
u64Buf
),
"
"
PRIu64
"
"
,
pRaftStore
->
currentTerm
);
cJSON_AddStringToObject
(
pRoot
,
"current_term"
,
u64Buf
);
snprintf
(
u64Buf
,
sizeof
(
u64Buf
),
"
%lu
"
,
pRaftStore
->
voteFor
.
addr
);
snprintf
(
u64Buf
,
sizeof
(
u64Buf
),
"
"
PRIu64
"
"
,
pRaftStore
->
voteFor
.
addr
);
cJSON_AddStringToObject
(
pRoot
,
"vote_for_addr"
,
u64Buf
);
cJSON_AddNumberToObject
(
pRoot
,
"vote_for_vgid"
,
pRaftStore
->
voteFor
.
vgId
);
...
...
@@ -142,11 +142,11 @@ int32_t raftStoreDeserialize(SRaftStore *pRaftStore, char *buf, size_t len) {
cJSON
*
pCurrentTerm
=
cJSON_GetObjectItem
(
pRoot
,
"current_term"
);
ASSERT
(
cJSON_IsString
(
pCurrentTerm
));
sscanf
(
pCurrentTerm
->
valuestring
,
"
%lu
"
,
&
(
pRaftStore
->
currentTerm
));
sscanf
(
pCurrentTerm
->
valuestring
,
"
"
PRIu64
"
"
,
&
(
pRaftStore
->
currentTerm
));
cJSON
*
pVoteForAddr
=
cJSON_GetObjectItem
(
pRoot
,
"vote_for_addr"
);
ASSERT
(
cJSON_IsString
(
pVoteForAddr
));
sscanf
(
pVoteForAddr
->
valuestring
,
"
%lu
"
,
&
(
pRaftStore
->
voteFor
.
addr
));
sscanf
(
pVoteForAddr
->
valuestring
,
"
"
PRIu64
"
"
,
&
(
pRaftStore
->
voteFor
.
addr
));
cJSON
*
pVoteForVgid
=
cJSON_GetObjectItem
(
pRoot
,
"vote_for_vgid"
);
pRaftStore
->
voteFor
.
vgId
=
pVoteForVgid
->
valueint
;
...
...
@@ -188,11 +188,11 @@ cJSON *raftStore2Json(SRaftStore *pRaftStore) {
cJSON
*
pRoot
=
cJSON_CreateObject
();
if
(
pRaftStore
!=
NULL
)
{
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"
%lu
"
,
pRaftStore
->
currentTerm
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"
"
PRIu64
"
"
,
pRaftStore
->
currentTerm
);
cJSON_AddStringToObject
(
pRoot
,
"currentTerm"
,
u64buf
);
cJSON
*
pVoteFor
=
cJSON_CreateObject
();
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"
%lu
"
,
pRaftStore
->
voteFor
.
addr
);
snprintf
(
u64buf
,
sizeof
(
u64buf
),
"
"
PRIu64
"
"
,
pRaftStore
->
voteFor
.
addr
);
cJSON_AddStringToObject
(
pVoteFor
,
"addr"
,
u64buf
);
{
uint64_t
u64
=
pRaftStore
->
voteFor
.
addr
;
...
...
@@ -216,7 +216,7 @@ cJSON *raftStore2Json(SRaftStore *pRaftStore) {
char
*
raftStore2Str
(
SRaftStore
*
pRaftStore
)
{
cJSON
*
pJson
=
raftStore2Json
(
pRaftStore
);
char
*
serialized
=
cJSON_Print
(
pJson
);
char
*
serialized
=
cJSON_Print
(
pJson
);
cJSON_Delete
(
pJson
);
return
serialized
;
}
...
...
@@ -224,25 +224,25 @@ char *raftStore2Str(SRaftStore *pRaftStore) {
// for debug -------------------
void
raftStorePrint
(
SRaftStore
*
pObj
)
{
char
*
serialized
=
raftStore2Str
(
pObj
);
printf
(
"raftStorePrint | len:
%lu
| %s
\n
"
,
strlen
(
serialized
),
serialized
);
printf
(
"raftStorePrint | len:
"
PRIu64
"
| %s
\n
"
,
strlen
(
serialized
),
serialized
);
fflush
(
NULL
);
taosMemoryFree
(
serialized
);
}
void
raftStorePrint2
(
char
*
s
,
SRaftStore
*
pObj
)
{
char
*
serialized
=
raftStore2Str
(
pObj
);
printf
(
"raftStorePrint2 | len:
%lu
| %s | %s
\n
"
,
strlen
(
serialized
),
s
,
serialized
);
printf
(
"raftStorePrint2 | len:
"
PRIu64
"
| %s | %s
\n
"
,
strlen
(
serialized
),
s
,
serialized
);
fflush
(
NULL
);
taosMemoryFree
(
serialized
);
}
void
raftStoreLog
(
SRaftStore
*
pObj
)
{
char
*
serialized
=
raftStore2Str
(
pObj
);
sTrace
(
"raftStoreLog | len:
%lu
| %s"
,
strlen
(
serialized
),
serialized
);
sTrace
(
"raftStoreLog | len:
"
PRIu64
"
| %s"
,
strlen
(
serialized
),
serialized
);
taosMemoryFree
(
serialized
);
}
void
raftStoreLog2
(
char
*
s
,
SRaftStore
*
pObj
)
{
char
*
serialized
=
raftStore2Str
(
pObj
);
sTrace
(
"raftStoreLog2 | len:
%lu
| %s | %s"
,
strlen
(
serialized
),
s
,
serialized
);
sTrace
(
"raftStoreLog2 | len:
"
PRIu64
"
| %s | %s"
,
strlen
(
serialized
),
s
,
serialized
);
taosMemoryFree
(
serialized
);
}
source/libs/sync/src/syncReplication.c
浏览文件 @
c2b348be
...
...
@@ -315,15 +315,18 @@ int32_t syncNodeAppendEntries(SSyncNode* pSyncNode, const SRaftId* destRaftId, c
int32_t
ret
=
0
;
do
{
char
host
[
128
];
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
destRaftId
->
addr
,
host
,
sizeof
(
host
),
&
port
);
sDebug
(
"vgId:%d, send sync-append-entries to %s:%d, {term:%"
PRIu64
", pre-index:%"
PRId64
", pre-term:%"
PRIu64
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"send sync-append-entries to %s:%d, {term:%"
PRIu64
", pre-index:%"
PRId64
", pre-term:%"
PRIu64
", pterm:%"
PRIu64
", commit:%"
PRId64
", "
"datalen:%d}"
,
pSyncNode
->
vgId
,
host
,
port
,
pMsg
->
term
,
pMsg
->
prevLogIndex
,
pMsg
->
prevLogTerm
,
pMsg
->
privateTerm
,
pMsg
->
commitIndex
,
pMsg
->
dataLen
);
host
,
port
,
pMsg
->
term
,
pMsg
->
prevLogIndex
,
pMsg
->
prevLogTerm
,
pMsg
->
privateTerm
,
pMsg
->
commitIndex
,
pMsg
->
dataLen
);
syncNodeEventLog
(
pSyncNode
,
logBuf
);
}
while
(
0
);
SRpcMsg
rpcMsg
;
...
...
@@ -335,13 +338,16 @@ int32_t syncNodeAppendEntries(SSyncNode* pSyncNode, const SRaftId* destRaftId, c
int32_t
syncNodeAppendEntriesBatch
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncAppendEntriesBatch
*
pMsg
)
{
do
{
char
host
[
128
];
char
host
[
64
];
uint16_t
port
;
syncUtilU642Addr
(
destRaftId
->
addr
,
host
,
sizeof
(
host
),
&
port
);
sDebug
(
"vgId:%d, send sync-append-entries-batch to %s:%d, {term:%"
PRIu64
", pre-index:%"
PRId64
", pre-term:%"
PRIu64
", pterm:%"
PRIu64
", commit:%"
PRId64
", datalen:%d, datacount:%d}"
,
char
logBuf
[
256
];
snprintf
(
logBuf
,
sizeof
(
logBuf
),
"send sync-append-entries-batch to %s:%d, {term:%"
PRIu64
", pre-index:%"
PRId64
", pre-term:%"
PRIu64
", pterm:%"
PRIu64
", commit:%"
PRId64
", datalen:%d, datacount:%d}"
,
pSyncNode
->
vgId
,
host
,
port
,
pMsg
->
term
,
pMsg
->
prevLogIndex
,
pMsg
->
prevLogTerm
,
pMsg
->
privateTerm
,
pMsg
->
commitIndex
,
pMsg
->
dataLen
,
pMsg
->
dataCount
);
syncNodeEventLog
(
pSyncNode
,
logBuf
);
}
while
(
0
);
SRpcMsg
rpcMsg
;
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录