Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
taosdata
TDengine
提交
49c19e13
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看板
提交
49c19e13
编写于
10月 18, 2022
作者:
M
Minghao Li
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
refactor(sync): rename function
上级
e810f2ad
变更
6
隐藏空白更改
内联
并排
Showing
6 changed file
with
43 addition
and
103 deletion
+43
-103
source/libs/sync/inc/syncElection.h
source/libs/sync/inc/syncElection.h
+2
-5
source/libs/sync/inc/syncReplication.h
source/libs/sync/inc/syncReplication.h
+6
-6
source/libs/sync/src/syncAppendEntriesReply.c
source/libs/sync/src/syncAppendEntriesReply.c
+1
-1
source/libs/sync/src/syncElection.c
source/libs/sync/src/syncElection.c
+26
-82
source/libs/sync/src/syncMain.c
source/libs/sync/src/syncMain.c
+3
-3
source/libs/sync/src/syncReplication.c
source/libs/sync/src/syncReplication.c
+5
-6
未找到文件。
source/libs/sync/inc/syncElection.h
浏览文件 @
49c19e13
...
...
@@ -37,13 +37,10 @@ extern "C" {
// msource |-> i,
// mdest |-> j])
// /\ UNCHANGED <<serverVars, candidateVars, leaderVars, logVars>>
//
int32_t
syncNodeRequestVotePeers
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeRequestVotePeersSnapshot
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeDoRequestVote
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeElect
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeRequestVote
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncRequestVote
*
pMsg
);
int32_t
syncNodeRequestVotePeers
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeSendRequestVote
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncRequestVote
*
pMsg
);
#ifdef __cplusplus
}
...
...
source/libs/sync/inc/syncReplication.h
浏览文件 @
49c19e13
...
...
@@ -50,15 +50,15 @@ extern "C" {
// msource |-> i,
// mdest |-> j])
// /\ UNCHANGED <<serverVars, candidateVars, leaderVars, logVars>>
//
int32_t
syncNodeHeartbeat
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncHeartbeat
*
pMsg
);
int32_t
syncNodeHeartbeatPeers
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeSendHeartbeat
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
pDestId
,
const
SyncHeartbeat
*
pMsg
);
int32_t
syncNodeDoReplicate
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeDoAppendEntries
(
SSyncNode
*
pSyncNode
,
SRaftId
*
pDestId
);
int32_t
syncNodeSendAppendEntries
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncAppendEntries
*
pMsg
);
int32_t
syncNodeMaybeSendAppendEntries
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncAppendEntries
*
pMsg
);
int32_t
syncNodeReplicate
(
SSyncNode
*
pSyncNode
);
int32_t
syncNodeReplicateOne
(
SSyncNode
*
pSyncNode
,
SRaftId
*
pDestId
);
int32_t
syncNodeSendAppendEntries
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
pDestId
,
const
SyncAppendEntries
*
pMsg
);
int32_t
syncNodeMaybeSendAppendEntries
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
pDestId
,
const
SyncAppendEntries
*
pMsg
);
#ifdef __cplusplus
}
...
...
source/libs/sync/src/syncAppendEntriesReply.c
浏览文件 @
49c19e13
...
...
@@ -129,7 +129,7 @@ int32_t syncNodeOnAppendEntriesReply(SSyncNode* ths, SyncAppendEntriesReply* pMs
ASSERT
(
pState
!=
NULL
);
if
(
pMsg
->
lastSendIndex
==
pState
->
lastSendIndex
)
{
syncNode
DoAppendEntries
(
ths
,
&
(
pMsg
->
srcId
));
syncNode
ReplicateOne
(
ths
,
&
(
pMsg
->
srcId
));
}
}
...
...
source/libs/sync/src/syncElection.c
浏览文件 @
49c19e13
...
...
@@ -30,68 +30,6 @@
// msource |-> i,
// mdest |-> j])
// /\ UNCHANGED <<serverVars, candidateVars, leaderVars, logVars>>
//
int32_t
syncNodeRequestVotePeers
(
SSyncNode
*
pSyncNode
)
{
ASSERT
(
pSyncNode
->
state
==
TAOS_SYNC_STATE_CANDIDATE
);
int32_t
ret
=
0
;
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
SyncRequestVote
*
pMsg
=
syncRequestVoteBuild
(
pSyncNode
->
vgId
);
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
destId
=
pSyncNode
->
peersId
[
i
];
pMsg
->
term
=
pSyncNode
->
pRaftStore
->
currentTerm
;
pMsg
->
lastLogIndex
=
pSyncNode
->
pLogStore
->
syncLogLastIndex
(
pSyncNode
->
pLogStore
);
pMsg
->
lastLogTerm
=
pSyncNode
->
pLogStore
->
syncLogLastTerm
(
pSyncNode
->
pLogStore
);
ret
=
syncNodeRequestVote
(
pSyncNode
,
&
pSyncNode
->
peersId
[
i
],
pMsg
);
ASSERT
(
ret
==
0
);
syncRequestVoteDestroy
(
pMsg
);
}
return
ret
;
}
int32_t
syncNodeRequestVotePeersSnapshot
(
SSyncNode
*
pSyncNode
)
{
ASSERT
(
pSyncNode
->
state
==
TAOS_SYNC_STATE_CANDIDATE
);
int32_t
ret
=
0
;
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
SyncRequestVote
*
pMsg
=
syncRequestVoteBuild
(
pSyncNode
->
vgId
);
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
destId
=
pSyncNode
->
peersId
[
i
];
pMsg
->
term
=
pSyncNode
->
pRaftStore
->
currentTerm
;
ret
=
syncNodeGetLastIndexTerm
(
pSyncNode
,
&
(
pMsg
->
lastLogIndex
),
&
(
pMsg
->
lastLogTerm
));
ASSERT
(
ret
==
0
);
ret
=
syncNodeRequestVote
(
pSyncNode
,
&
pSyncNode
->
peersId
[
i
],
pMsg
);
ASSERT
(
ret
==
0
);
syncRequestVoteDestroy
(
pMsg
);
}
return
ret
;
}
int32_t
syncNodeDoRequestVote
(
SSyncNode
*
pSyncNode
)
{
if
(
pSyncNode
->
state
!=
TAOS_SYNC_STATE_CANDIDATE
)
{
syncNodeEventLog
(
pSyncNode
,
"not candidate, stop elect"
);
return
0
;
}
int32_t
ret
=
0
;
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
SyncRequestVote
*
pMsg
=
syncRequestVoteBuild
(
pSyncNode
->
vgId
);
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
destId
=
pSyncNode
->
peersId
[
i
];
pMsg
->
term
=
pSyncNode
->
pRaftStore
->
currentTerm
;
ret
=
syncNodeGetLastIndexTerm
(
pSyncNode
,
&
(
pMsg
->
lastLogIndex
),
&
(
pMsg
->
lastLogTerm
));
ASSERT
(
ret
==
0
);
ret
=
syncNodeRequestVote
(
pSyncNode
,
&
pSyncNode
->
peersId
[
i
],
pMsg
);
ASSERT
(
ret
==
0
);
syncRequestVoteDestroy
(
pMsg
);
}
return
ret
;
}
int32_t
syncNodeElect
(
SSyncNode
*
pSyncNode
)
{
syncNodeEventLog
(
pSyncNode
,
"begin election"
);
...
...
@@ -121,32 +59,38 @@ int32_t syncNodeElect(SSyncNode* pSyncNode) {
return
ret
;
}
if
(
syncNodeIsMnode
(
pSyncNode
))
{
switch
(
pSyncNode
->
pRaftCfg
->
snapshotStrategy
)
{
case
SYNC_STRATEGY_NO_SNAPSHOT
:
ret
=
syncNodeRequestVotePeers
(
pSyncNode
);
break
;
case
SYNC_STRATEGY_STANDARD_SNAPSHOT
:
case
SYNC_STRATEGY_WAL_FIRST
:
ret
=
syncNodeRequestVotePeersSnapshot
(
pSyncNode
);
break
;
default:
ret
=
syncNodeRequestVotePeers
(
pSyncNode
);
break
;
}
}
else
{
ret
=
syncNodeDoRequestVote
(
pSyncNode
);
}
ret
=
syncNodeRequestVotePeers
(
pSyncNode
);
ASSERT
(
ret
==
0
);
syncNodeResetElectTimer
(
pSyncNode
);
return
ret
;
}
int32_t
syncNodeRequestVote
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncRequestVote
*
pMsg
)
{
int32_t
syncNodeRequestVotePeers
(
SSyncNode
*
pSyncNode
)
{
if
(
pSyncNode
->
state
!=
TAOS_SYNC_STATE_CANDIDATE
)
{
syncNodeEventLog
(
pSyncNode
,
"not candidate, stop elect"
);
return
0
;
}
int32_t
ret
=
0
;
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
SyncRequestVote
*
pMsg
=
syncRequestVoteBuild
(
pSyncNode
->
vgId
);
pMsg
->
srcId
=
pSyncNode
->
myRaftId
;
pMsg
->
destId
=
pSyncNode
->
peersId
[
i
];
pMsg
->
term
=
pSyncNode
->
pRaftStore
->
currentTerm
;
ret
=
syncNodeGetLastIndexTerm
(
pSyncNode
,
&
(
pMsg
->
lastLogIndex
),
&
(
pMsg
->
lastLogTerm
));
ASSERT
(
ret
==
0
);
ret
=
syncNodeSendRequestVote
(
pSyncNode
,
&
pSyncNode
->
peersId
[
i
],
pMsg
);
ASSERT
(
ret
==
0
);
syncRequestVoteDestroy
(
pMsg
);
}
return
ret
;
}
int32_t
syncNodeSendRequestVote
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncRequestVote
*
pMsg
)
{
int32_t
ret
=
0
;
syncLogSendRequestVote
(
pSyncNode
,
pMsg
,
""
);
...
...
source/libs/sync/src/syncMain.c
浏览文件 @
49c19e13
...
...
@@ -2299,7 +2299,7 @@ void syncNodeCandidate2Leader(SSyncNode* pSyncNode) {
syncMaybeAdvanceCommitIndex
(
pSyncNode
);
if
(
pSyncNode
->
replicaNum
>
1
)
{
syncNode
Do
Replicate
(
pSyncNode
);
syncNodeReplicate
(
pSyncNode
);
}
}
...
...
@@ -2696,7 +2696,7 @@ static void syncNodeEqPeerHeartbeatTimer(void* param, void* tmrId) {
#endif
// send msg
syncNodeHeartbeat
(
pSyncNode
,
&
(
pSyncMsg
->
destId
),
pSyncMsg
);
syncNode
Send
Heartbeat
(
pSyncNode
,
&
(
pSyncMsg
->
destId
),
pSyncMsg
);
syncHeartbeatDestroy
(
pSyncMsg
);
...
...
@@ -2900,7 +2900,7 @@ int32_t syncNodeOnClientRequest(SSyncNode* ths, SyncClientRequest* pMsg, SyncInd
// if mulit replica, start replicate right now
if
(
ths
->
replicaNum
>
1
)
{
syncNode
Do
Replicate
(
ths
);
syncNodeReplicate
(
ths
);
}
// if only myself, maybe commit right now
...
...
source/libs/sync/src/syncReplication.c
浏览文件 @
49c19e13
...
...
@@ -47,9 +47,8 @@
// msource |-> i,
// mdest |-> j])
// /\ UNCHANGED <<serverVars, candidateVars, leaderVars, logVars>>
//
int32_t
syncNode
DoAppendEntries
(
SSyncNode
*
pSyncNode
,
SRaftId
*
pDestId
)
{
int32_t
syncNode
ReplicateOne
(
SSyncNode
*
pSyncNode
,
SRaftId
*
pDestId
)
{
// next index
SyncIndex
nextIndex
=
syncIndexMgrGetIndex
(
pSyncNode
->
pNextIndex
,
pDestId
);
...
...
@@ -118,7 +117,7 @@ int32_t syncNodeDoAppendEntries(SSyncNode* pSyncNode, SRaftId* pDestId) {
return
0
;
}
int32_t
syncNode
Do
Replicate
(
SSyncNode
*
pSyncNode
)
{
int32_t
syncNodeReplicate
(
SSyncNode
*
pSyncNode
)
{
syncNodeEventLog
(
pSyncNode
,
"do replicate"
);
if
(
pSyncNode
->
state
!=
TAOS_SYNC_STATE_LEADER
)
{
...
...
@@ -128,7 +127,7 @@ int32_t syncNodeDoReplicate(SSyncNode* pSyncNode) {
int32_t
ret
=
0
;
for
(
int
i
=
0
;
i
<
pSyncNode
->
peersNum
;
++
i
)
{
SRaftId
*
pDestId
=
&
(
pSyncNode
->
peersId
[
i
]);
ret
=
syncNode
DoAppendEntries
(
pSyncNode
,
pDestId
);
ret
=
syncNode
ReplicateOne
(
pSyncNode
,
pDestId
);
if
(
ret
!=
0
)
{
char
host
[
64
];
int16_t
port
;
...
...
@@ -175,7 +174,7 @@ int32_t syncNodeAppendEntries(SSyncNode* pSyncNode, const SRaftId* destRaftId, c
return
ret
;
}
int32_t
syncNodeHeartbeat
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncHeartbeat
*
pMsg
)
{
int32_t
syncNode
Send
Heartbeat
(
SSyncNode
*
pSyncNode
,
const
SRaftId
*
destRaftId
,
const
SyncHeartbeat
*
pMsg
)
{
int32_t
ret
=
0
;
syncLogSendHeartbeat
(
pSyncNode
,
pMsg
,
""
);
...
...
@@ -198,7 +197,7 @@ int32_t syncNodeHeartbeatPeers(SSyncNode* pSyncNode) {
syncHeartbeat2RpcMsg
(
pSyncMsg
,
&
rpcMsg
);
// send msg
syncNodeHeartbeat
(
pSyncNode
,
&
(
pSyncMsg
->
destId
),
pSyncMsg
);
syncNode
Send
Heartbeat
(
pSyncNode
,
&
(
pSyncMsg
->
destId
),
pSyncMsg
);
syncHeartbeatDestroy
(
pSyncMsg
);
}
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录