Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
taosdata
TDengine
提交
09786a12
T
TDengine
项目概览
taosdata
/
TDengine
1 年多 前同步成功
通知
1185
Star
22017
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看板
提交
09786a12
编写于
4月 11, 2023
作者:
B
Benguang Zhao
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
enh: refactor func names doOnce, attempt, probe, and sendTo of syncLogRepl
上级
f6be5f2c
变更
3
隐藏空白更改
内联
并排
Showing
3 changed file
with
21 addition
and
19 deletion
+21
-19
source/libs/sync/inc/syncPipeline.h
source/libs/sync/inc/syncPipeline.h
+8
-6
source/libs/sync/src/syncPipeline.c
source/libs/sync/src/syncPipeline.c
+12
-12
source/libs/sync/src/syncReplication.c
source/libs/sync/src/syncReplication.c
+1
-1
未找到文件。
source/libs/sync/inc/syncPipeline.h
浏览文件 @
09786a12
...
...
@@ -77,18 +77,19 @@ static FORCE_INLINE int32_t syncLogReplGetNextRetryBackoff(SSyncLogReplMgr* pMgr
SyncTerm
syncLogReplGetPrevLogTerm
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
);
int32_t
syncLogReplReplicateOnce
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
);
int32_t
syncLogReplReplicateOneTo
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
,
SyncTerm
*
pTerm
,
SRaftId
*
pDestId
,
bool
*
pBarrier
);
int32_t
syncLogReplReplicateAttempt
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
);
int32_t
syncLogReplReplicateProbe
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
);
int32_t
syncLogReplDoOnce
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
);
int32_t
syncLogReplAttempt
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
);
int32_t
syncLogReplProbe
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
);
int32_t
syncLogReplRetryOnNeed
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
);
int32_t
syncLogReplSendTo
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
,
SyncTerm
*
pTerm
,
SRaftId
*
pDestId
,
bool
*
pBarrier
);
int32_t
syncLogReplProcessReply
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncAppendEntriesReply
*
pMsg
);
int32_t
syncLogReplProcessReplyAsRecovery
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncAppendEntriesReply
*
pMsg
);
int32_t
syncLogReplProcessReplyAsNormal
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncAppendEntriesReply
*
pMsg
);
int32_t
syncLogReplProcessHeartbeatReply
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncHeartbeatReply
*
pMsg
);
int32_t
syncLogReplRetryOnNeed
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
);
// SSyncLogBuffer
SSyncLogBuffer
*
syncLogBufferCreate
();
...
...
@@ -100,6 +101,7 @@ int32_t syncLogBufferReInit(SSyncLogBuffer* pBuf, SSyncNode* pNode);
int64_t
syncLogBufferGetEndIndex
(
SSyncLogBuffer
*
pBuf
);
SyncTerm
syncLogBufferGetLastMatchTerm
(
SSyncLogBuffer
*
pBuf
);
bool
syncLogBufferIsEmpty
(
SSyncLogBuffer
*
pBuf
);
int32_t
syncLogBufferAppend
(
SSyncLogBuffer
*
pBuf
,
SSyncNode
*
pNode
,
SSyncRaftEntry
*
pEntry
);
int32_t
syncLogBufferAccept
(
SSyncLogBuffer
*
pBuf
,
SSyncNode
*
pNode
,
SSyncRaftEntry
*
pEntry
,
SyncTerm
prevTerm
);
int64_t
syncLogBufferProceed
(
SSyncLogBuffer
*
pBuf
,
SSyncNode
*
pNode
,
SyncTerm
*
pMatchTerm
);
...
...
source/libs/sync/src/syncPipeline.c
浏览文件 @
09786a12
...
...
@@ -666,7 +666,7 @@ int32_t syncLogReplRetryOnNeed(SSyncLogReplMgr* pMgr, SSyncNode* pNode) {
}
bool
barrier
=
false
;
if
(
syncLogRepl
ReplicateOne
To
(
pMgr
,
pNode
,
index
,
&
term
,
pDestId
,
&
barrier
)
<
0
)
{
if
(
syncLogRepl
Send
To
(
pMgr
,
pNode
,
index
,
&
term
,
pDestId
,
&
barrier
)
<
0
)
{
sError
(
"vgId:%d, failed to replicate sync log entry since %s. index:%"
PRId64
", dest:%"
PRIx64
""
,
pNode
->
vgId
,
terrstr
(),
index
,
pDestId
->
addr
);
goto
_out
;
...
...
@@ -774,7 +774,7 @@ int32_t syncLogReplProcessReplyAsRecovery(SSyncLogReplMgr* pMgr, SSyncNode* pNod
// attempt to replicate the raft log at index
(
void
)
syncLogReplReset
(
pMgr
);
return
syncLogRepl
Replicate
Probe
(
pMgr
,
pNode
,
index
);
return
syncLogReplProbe
(
pMgr
,
pNode
,
index
);
}
int32_t
syncLogReplProcessHeartbeatReply
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncHeartbeatReply
*
pMsg
)
{
...
...
@@ -809,16 +809,16 @@ int32_t syncLogReplProcessReply(SSyncLogReplMgr* pMgr, SSyncNode* pNode, SyncApp
return
0
;
}
int32_t
syncLogRepl
Replicate
Once
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
)
{
int32_t
syncLogRepl
Do
Once
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
)
{
if
(
pMgr
->
restored
)
{
(
void
)
syncLogRepl
Replicate
Attempt
(
pMgr
,
pNode
);
(
void
)
syncLogReplAttempt
(
pMgr
,
pNode
);
}
else
{
(
void
)
syncLogRepl
Replicate
Probe
(
pMgr
,
pNode
,
pNode
->
pLogBuf
->
matchIndex
);
(
void
)
syncLogReplProbe
(
pMgr
,
pNode
,
pNode
->
pLogBuf
->
matchIndex
);
}
return
0
;
}
int32_t
syncLogRepl
Replicate
Probe
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
)
{
int32_t
syncLogReplProbe
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
)
{
ASSERT
(
!
pMgr
->
restored
);
ASSERT
(
pMgr
->
startIndex
>=
0
);
int64_t
retryMaxWaitMs
=
syncGetRetryMaxWaitMs
();
...
...
@@ -833,7 +833,7 @@ int32_t syncLogReplReplicateProbe(SSyncLogReplMgr* pMgr, SSyncNode* pNode, SyncI
SRaftId
*
pDestId
=
&
pNode
->
replicasId
[
pMgr
->
peerId
];
bool
barrier
=
false
;
SyncTerm
term
=
-
1
;
if
(
syncLogRepl
ReplicateOne
To
(
pMgr
,
pNode
,
index
,
&
term
,
pDestId
,
&
barrier
)
<
0
)
{
if
(
syncLogRepl
Send
To
(
pMgr
,
pNode
,
index
,
&
term
,
pDestId
,
&
barrier
)
<
0
)
{
sError
(
"vgId:%d, failed to replicate log entry since %s. index:%"
PRId64
", dest: 0x%016"
PRIx64
""
,
pNode
->
vgId
,
terrstr
(),
index
,
pDestId
->
addr
);
return
-
1
;
...
...
@@ -856,7 +856,7 @@ int32_t syncLogReplReplicateProbe(SSyncLogReplMgr* pMgr, SSyncNode* pNode, SyncI
return
0
;
}
int32_t
syncLogRepl
Replicate
Attempt
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
)
{
int32_t
syncLogReplAttempt
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
)
{
ASSERT
(
pMgr
->
restored
);
SRaftId
*
pDestId
=
&
pNode
->
replicasId
[
pMgr
->
peerId
];
...
...
@@ -878,7 +878,7 @@ int32_t syncLogReplReplicateAttempt(SSyncLogReplMgr* pMgr, SSyncNode* pNode) {
SRaftId
*
pDestId
=
&
pNode
->
replicasId
[
pMgr
->
peerId
];
bool
barrier
=
false
;
SyncTerm
term
=
-
1
;
if
(
syncLogRepl
ReplicateOne
To
(
pMgr
,
pNode
,
index
,
&
term
,
pDestId
,
&
barrier
)
<
0
)
{
if
(
syncLogRepl
Send
To
(
pMgr
,
pNode
,
index
,
&
term
,
pDestId
,
&
barrier
)
<
0
)
{
sError
(
"vgId:%d, failed to replicate log entry since %s. index:%"
PRId64
", dest: 0x%016"
PRIx64
""
,
pNode
->
vgId
,
terrstr
(),
index
,
pDestId
->
addr
);
return
-
1
;
...
...
@@ -931,7 +931,7 @@ int32_t syncLogReplProcessReplyAsNormal(SSyncLogReplMgr* pMgr, SSyncNode* pNode,
pMgr
->
startIndex
=
pMgr
->
matchIndex
;
}
return
syncLogRepl
Replicate
Attempt
(
pMgr
,
pNode
);
return
syncLogReplAttempt
(
pMgr
,
pNode
);
}
SSyncLogReplMgr
*
syncLogReplCreate
()
{
...
...
@@ -1126,8 +1126,8 @@ SSyncRaftEntry* syncLogBufferGetOneEntry(SSyncLogBuffer* pBuf, SSyncNode* pNode,
return
pEntry
;
}
int32_t
syncLogRepl
ReplicateOneTo
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
,
SyncTerm
*
pTerm
,
SRaftId
*
pDestId
,
bool
*
pBarrier
)
{
int32_t
syncLogRepl
SendTo
(
SSyncLogReplMgr
*
pMgr
,
SSyncNode
*
pNode
,
SyncIndex
index
,
SyncTerm
*
pTerm
,
SRaftId
*
pDestId
,
bool
*
pBarrier
)
{
SSyncRaftEntry
*
pEntry
=
NULL
;
SRpcMsg
msgOut
=
{
0
};
bool
inBuf
=
false
;
...
...
source/libs/sync/src/syncReplication.c
浏览文件 @
09786a12
...
...
@@ -74,7 +74,7 @@ int32_t syncNodeReplicateWithoutLock(SSyncNode* pNode) {
continue
;
}
SSyncLogReplMgr
*
pMgr
=
pNode
->
logReplMgrs
[
i
];
(
void
)
syncLogRepl
Replicate
Once
(
pMgr
,
pNode
);
(
void
)
syncLogRepl
Do
Once
(
pMgr
,
pNode
);
}
return
0
;
}
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录