• Home
  • Features
  • Pricing
  • Docs
  • Announcements
  • Sign In

taosdata / TDengine / #3559

18 Dec 2024 12:59AM UTC coverage: 59.805% (+0.03%) from 59.778%
#3559

push

travis-ci

web-flow
Merge pull request #29187 from taosdata/merge/mainto3.0

merge: main to 3.0 branch

132705 of 287544 branches covered (46.15%)

Branch coverage included in aggregate %.

87 of 95 new or added lines in 19 files covered. (91.58%)

1132 existing lines in 133 files now uncovered.

209591 of 284807 relevant lines covered (73.59%)

8125235.78 hits per line

Source File
Press 'n' to go to next uncovered line, 'b' for previous

51.47
/source/libs/sync/src/syncMain.c
1
/*
2
 * Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
3
 *
4
 * This program is free software: you can use, redistribute, and/or modify
5
 * it under the terms of the GNU Affero General Public License, version 3
6
 * or later ("AGPL"), as published by the Free Software Foundation.
7
 *
8
 * This program is distributed in the hope that it will be useful, but WITHOUT
9
 * ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
10
 * FITNESS FOR A PARTICULAR PURPOSE.
11
 *
12
 * You should have received a copy of the GNU Affero General Public License
13
 * along with this program. If not, see <http://www.gnu.org/licenses/>.
14
 */
15

16
#define _DEFAULT_SOURCE
17
#include "sync.h"
18
#include "syncAppendEntries.h"
19
#include "syncAppendEntriesReply.h"
20
#include "syncCommit.h"
21
#include "syncElection.h"
22
#include "syncEnv.h"
23
#include "syncIndexMgr.h"
24
#include "syncInt.h"
25
#include "syncMessage.h"
26
#include "syncPipeline.h"
27
#include "syncRaftCfg.h"
28
#include "syncRaftLog.h"
29
#include "syncRaftStore.h"
30
#include "syncReplication.h"
31
#include "syncRequestVote.h"
32
#include "syncRequestVoteReply.h"
33
#include "syncRespMgr.h"
34
#include "syncSnapshot.h"
35
#include "syncTimeout.h"
36
#include "syncUtil.h"
37
#include "syncVoteMgr.h"
38
#include "tglobal.h"
39
#include "tmisce.h"
40
#include "tref.h"
41

42
static void    syncNodeEqPingTimer(void* param, void* tmrId);
43
static void    syncNodeEqElectTimer(void* param, void* tmrId);
44
static void    syncNodeEqHeartbeatTimer(void* param, void* tmrId);
45
static int32_t syncNodeAppendNoop(SSyncNode* ths);
46
static void    syncNodeEqPeerHeartbeatTimer(void* param, void* tmrId);
47
static bool    syncIsConfigChanged(const SSyncCfg* pOldCfg, const SSyncCfg* pNewCfg);
48
static int32_t syncHbTimerInit(SSyncNode* pSyncNode, SSyncTimer* pSyncTimer, SRaftId destId);
49
static int32_t syncHbTimerStart(SSyncNode* pSyncNode, SSyncTimer* pSyncTimer);
50
static int32_t syncHbTimerStop(SSyncNode* pSyncNode, SSyncTimer* pSyncTimer);
51
static int32_t syncNodeUpdateNewConfigIndex(SSyncNode* ths, SSyncCfg* pNewCfg);
52
static bool    syncNodeInConfig(SSyncNode* pSyncNode, const SSyncCfg* config);
53
static int32_t syncNodeDoConfigChange(SSyncNode* pSyncNode, SSyncCfg* newConfig, SyncIndex lastConfigChangeIndex);
54
static bool    syncNodeIsOptimizedOneReplica(SSyncNode* ths, SRpcMsg* pMsg);
55

56
static bool    syncNodeCanChange(SSyncNode* pSyncNode);
57
static int32_t syncNodeLeaderTransfer(SSyncNode* pSyncNode);
58
static int32_t syncNodeLeaderTransferTo(SSyncNode* pSyncNode, SNodeInfo newLeader);
59
static int32_t syncDoLeaderTransfer(SSyncNode* ths, SRpcMsg* pRpcMsg, SSyncRaftEntry* pEntry);
60

61
static ESyncStrategy syncNodeStrategy(SSyncNode* pSyncNode);
62

63
int64_t syncOpen(SSyncInfo* pSyncInfo, int32_t vnodeVersion) {
11,569✔
64
  sInfo("vgId:%d, start to open sync", pSyncInfo->vgId);
11,569✔
65
  SSyncNode* pSyncNode = syncNodeOpen(pSyncInfo, vnodeVersion);
11,569✔
66
  if (pSyncNode == NULL) {
11,570!
67
    sError("vgId:%d, failed to open sync node", pSyncInfo->vgId);
×
68
    return -1;
×
69
  }
70

71
  pSyncNode->rid = syncNodeAdd(pSyncNode);
11,570✔
72
  if (pSyncNode->rid < 0) {
11,570!
73
    syncNodeClose(pSyncNode);
×
74
    return -1;
×
75
  }
76

77
  pSyncNode->pingBaseLine = pSyncInfo->pingMs;
11,570✔
78
  pSyncNode->pingTimerMS = pSyncInfo->pingMs;
11,570✔
79
  pSyncNode->electBaseLine = pSyncInfo->electMs;
11,570✔
80
  pSyncNode->hbBaseLine = pSyncInfo->heartbeatMs;
11,570✔
81
  pSyncNode->heartbeatTimerMS = pSyncInfo->heartbeatMs;
11,570✔
82
  pSyncNode->msgcb = pSyncInfo->msgcb;
11,570✔
83
  sInfo("vgId:%d, sync opened", pSyncInfo->vgId);
11,570✔
84
  return pSyncNode->rid;
11,570✔
85
}
86

87
int32_t syncStart(int64_t rid) {
11,569✔
88
  int32_t    code = 0;
11,569✔
89
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
11,569✔
90
  if (pSyncNode == NULL) {
11,569!
91
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
92
    if (terrno != 0) code = terrno;
×
93
    sError("failed to acquire rid:%" PRId64 " of tsNodeReftId for pSyncNode", rid);
×
94
    TAOS_RETURN(code);
×
95
  }
96
  sInfo("vgId:%d, begin to start sync", pSyncNode->vgId);
11,569✔
97

98
  if ((code = syncNodeRestore(pSyncNode)) < 0) {
11,569!
99
    sError("vgId:%d, failed to restore sync log buffer since %s", pSyncNode->vgId, tstrerror(code));
×
100
    goto _err;
×
101
  }
102

103
  if ((code = syncNodeStart(pSyncNode)) < 0) {
11,569✔
104
    sError("vgId:%d, failed to start sync node since %s", pSyncNode->vgId, tstrerror(code));
1!
105
    goto _err;
1✔
106
  }
107

108
  syncNodeRelease(pSyncNode);
11,568✔
109

110
  sInfo("vgId:%d, sync started", pSyncNode->vgId);
11,568✔
111

112
  TAOS_RETURN(code);
11,568✔
113

114
_err:
1✔
115
  syncNodeRelease(pSyncNode);
1✔
116
  TAOS_RETURN(code);
1✔
117
}
118

119
int32_t syncNodeGetConfig(int64_t rid, SSyncCfg* cfg) {
13,992✔
120
  int32_t    code = 0;
13,992✔
121
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
13,992✔
122

123
  if (pSyncNode == NULL) {
13,992!
124
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
125
    if (terrno != 0) code = terrno;
×
126
    sError("failed to acquire rid:%" PRId64 " of tsNodeReftId for pSyncNode", rid);
×
127
    TAOS_RETURN(code);
×
128
  }
129

130
  *cfg = pSyncNode->raftCfg.cfg;
13,992✔
131

132
  syncNodeRelease(pSyncNode);
13,992✔
133

134
  return 0;
13,992✔
135
}
136

137
void syncStop(int64_t rid) {
11,569✔
138
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
11,569✔
139
  if (pSyncNode != NULL) {
11,569!
140
    pSyncNode->isStart = false;
11,569✔
141
    syncNodeRelease(pSyncNode);
11,569✔
142
    syncNodeRemove(rid);
11,569✔
143
  }
144
}
11,568✔
145

146
void syncPreStop(int64_t rid) {
11,569✔
147
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
11,569✔
148
  if (pSyncNode != NULL) {
11,569!
149
    if (snapshotReceiverIsStart(pSyncNode->pNewNodeReceiver)) {
11,569!
150
      sInfo("vgId:%d, stop snapshot receiver", pSyncNode->vgId);
×
151
      snapshotReceiverStop(pSyncNode->pNewNodeReceiver);
×
152
    }
153
    syncNodePreClose(pSyncNode);
11,569✔
154
    syncNodeRelease(pSyncNode);
11,569✔
155
  }
156
}
11,569✔
157

158
void syncPostStop(int64_t rid) {
10,052✔
159
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
10,052✔
160
  if (pSyncNode != NULL) {
10,052!
161
    syncNodePostClose(pSyncNode);
10,052✔
162
    syncNodeRelease(pSyncNode);
10,052✔
163
  }
164
}
10,052✔
165

166
static bool syncNodeCheckNewConfig(SSyncNode* pSyncNode, const SSyncCfg* pCfg) {
1,421✔
167
  if (!syncNodeInConfig(pSyncNode, pCfg)) return false;
1,421!
168
  return abs(pCfg->replicaNum - pSyncNode->replicaNum) <= 1;
1,421!
169
}
170

171
int32_t syncReconfig(int64_t rid, SSyncCfg* pNewCfg) {
1,508✔
172
  int32_t    code = 0;
1,508✔
173
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
1,508✔
174
  if (pSyncNode == NULL) {
1,508!
175
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
176
    if (terrno != 0) code = terrno;
×
177
    TAOS_RETURN(code);
×
178
  }
179

180
  if (pSyncNode->raftCfg.lastConfigIndex >= pNewCfg->lastIndex) {
1,508✔
181
    syncNodeRelease(pSyncNode);
87✔
182
    sInfo("vgId:%d, no need Reconfig, current index:%" PRId64 ", new index:%" PRId64, pSyncNode->vgId,
87!
183
          pSyncNode->raftCfg.lastConfigIndex, pNewCfg->lastIndex);
184
    return 0;
87✔
185
  }
186

187
  if (!syncNodeCheckNewConfig(pSyncNode, pNewCfg)) {
1,421!
188
    syncNodeRelease(pSyncNode);
×
189
    code = TSDB_CODE_SYN_NEW_CONFIG_ERROR;
×
190
    sError("vgId:%d, failed to reconfig since invalid new config", pSyncNode->vgId);
×
191
    TAOS_RETURN(code);
×
192
  }
193

194
  TAOS_CHECK_RETURN(syncNodeUpdateNewConfigIndex(pSyncNode, pNewCfg));
1,421!
195

196
  if (syncNodeDoConfigChange(pSyncNode, pNewCfg, pNewCfg->lastIndex) != 0) {
1,421!
197
    code = TSDB_CODE_SYN_NEW_CONFIG_ERROR;
×
198
    sError("vgId:%d, failed to reconfig since do change error", pSyncNode->vgId);
×
199
    TAOS_RETURN(code);
×
200
  }
201

202
  if (pSyncNode->state == TAOS_SYNC_STATE_LEADER || pSyncNode->state == TAOS_SYNC_STATE_ASSIGNED_LEADER) {
1,421!
203
    // TODO check return value
204
    TAOS_CHECK_RETURN(syncNodeStopHeartbeatTimer(pSyncNode));
1,277!
205

206
    for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
20,432✔
207
      TAOS_CHECK_RETURN(syncHbTimerInit(pSyncNode, &pSyncNode->peerHeartbeatTimerArr[i], pSyncNode->replicasId[i]));
19,155!
208
    }
209

210
    TAOS_CHECK_RETURN(syncNodeStartHeartbeatTimer(pSyncNode));
1,277!
211
    // syncNodeReplicate(pSyncNode);
212
  }
213

214
  syncNodeRelease(pSyncNode);
1,421✔
215
  TAOS_RETURN(code);
1,421✔
216
}
217

218
int32_t syncProcessMsg(int64_t rid, SRpcMsg* pMsg) {
5,888,039✔
219
  int32_t code = -1;
5,888,039✔
220
  if (!syncIsInit()) {
5,888,039!
221
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
222
    if (terrno != 0) code = terrno;
×
223
    TAOS_RETURN(code);
×
224
  }
225

226
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
5,888,053✔
227
  if (pSyncNode == NULL) {
5,888,173✔
228
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
101✔
229
    if (terrno != 0) code = terrno;
101!
230
    TAOS_RETURN(code);
×
231
  }
232

233
  switch (pMsg->msgType) {
5,888,072!
234
    case TDMT_SYNC_HEARTBEAT:
30,440✔
235
      code = syncNodeOnHeartbeat(pSyncNode, pMsg);
30,440✔
236
      break;
30,440✔
237
    case TDMT_SYNC_HEARTBEAT_REPLY:
30,223✔
238
      code = syncNodeOnHeartbeatReply(pSyncNode, pMsg);
30,223✔
239
      break;
30,223✔
240
    case TDMT_SYNC_TIMEOUT:
44,155✔
241
      code = syncNodeOnTimeout(pSyncNode, pMsg);
44,155✔
242
      break;
44,189✔
243
    case TDMT_SYNC_TIMEOUT_ELECTION:
964✔
244
      code = syncNodeOnTimeout(pSyncNode, pMsg);
964✔
245
      break;
964✔
246
    case TDMT_SYNC_CLIENT_REQUEST:
205,262✔
247
      code = syncNodeOnClientRequest(pSyncNode, pMsg, NULL);
205,262✔
248
      break;
205,265✔
249
    case TDMT_SYNC_REQUEST_VOTE:
1,668✔
250
      code = syncNodeOnRequestVote(pSyncNode, pMsg);
1,668✔
251
      break;
1,668✔
252
    case TDMT_SYNC_REQUEST_VOTE_REPLY:
1,632✔
253
      code = syncNodeOnRequestVoteReply(pSyncNode, pMsg);
1,632✔
254
      break;
1,632✔
255
    case TDMT_SYNC_APPEND_ENTRIES:
2,735,062✔
256
      code = syncNodeOnAppendEntries(pSyncNode, pMsg);
2,735,062✔
257
      break;
2,735,066✔
258
    case TDMT_SYNC_APPEND_ENTRIES_REPLY:
2,733,338✔
259
      code = syncNodeOnAppendEntriesReply(pSyncNode, pMsg);
2,733,338✔
260
      break;
2,733,337✔
261
    case TDMT_SYNC_SNAPSHOT_SEND:
37,454✔
262
      code = syncNodeOnSnapshot(pSyncNode, pMsg);
37,454✔
263
      break;
37,454✔
264
    case TDMT_SYNC_SNAPSHOT_RSP:
37,506✔
265
      code = syncNodeOnSnapshotRsp(pSyncNode, pMsg);
37,506✔
266
      break;
37,506✔
267
    case TDMT_SYNC_LOCAL_CMD:
30,364✔
268
      code = syncNodeOnLocalCmd(pSyncNode, pMsg);
30,364✔
269
      break;
30,364✔
270
    case TDMT_SYNC_FORCE_FOLLOWER:
4✔
271
      code = syncForceBecomeFollower(pSyncNode, pMsg);
4✔
272
      break;
4✔
273
    case TDMT_SYNC_SET_ASSIGNED_LEADER:
×
274
      code = syncBecomeAssignedLeader(pSyncNode, pMsg);
×
275
      break;
×
276
    default:
×
277
      code = TSDB_CODE_MSG_NOT_PROCESSED;
×
278
  }
279

280
  syncNodeRelease(pSyncNode);
5,888,112✔
281
  if (code != 0) {
5,888,100✔
282
    sDebug("vgId:%d, failed to process sync msg:%p type:%s since %s", pSyncNode->vgId, pMsg, TMSG_INFO(pMsg->msgType),
15!
283
           tstrerror(code));
284
  }
285
  TAOS_RETURN(code);
5,888,100✔
286
}
287

288
int32_t syncLeaderTransfer(int64_t rid) {
11,569✔
289
  int32_t    code = 0;
11,569✔
290
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
11,569✔
291
  if (pSyncNode == NULL) {
11,569!
292
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
293
    if (terrno != 0) code = terrno;
×
294
    TAOS_RETURN(code);
×
295
  }
296

297
  int32_t ret = syncNodeLeaderTransfer(pSyncNode);
11,569✔
298
  syncNodeRelease(pSyncNode);
11,569✔
299
  return ret;
11,569✔
300
}
301

302
int32_t syncForceBecomeFollower(SSyncNode* ths, const SRpcMsg* pRpcMsg) {
4✔
303
  SRaftId id = {0};
4✔
304
  syncNodeBecomeFollower(ths, id, "force election");
4✔
305

306
  SRpcMsg rsp = {
4✔
307
      .code = 0,
308
      .pCont = pRpcMsg->info.rsp,
4✔
309
      .contLen = pRpcMsg->info.rspLen,
4✔
310
      .info = pRpcMsg->info,
311
  };
312
  tmsgSendRsp(&rsp);
4✔
313

314
  return 0;
4✔
315
}
316

317
int32_t syncBecomeAssignedLeader(SSyncNode* ths, SRpcMsg* pRpcMsg) {
×
318
  int32_t code = TSDB_CODE_MND_ARB_TOKEN_MISMATCH;
×
319
  void*   pHead = NULL;
×
320
  int32_t contLen = 0;
×
321

322
  SVArbSetAssignedLeaderReq req = {0};
×
323
  if (tDeserializeSVArbSetAssignedLeaderReq((char*)pRpcMsg->pCont + sizeof(SMsgHead), pRpcMsg->contLen, &req) != 0) {
×
324
    sError("vgId:%d, failed to deserialize SVArbSetAssignedLeaderReq", ths->vgId);
×
325
    code = TSDB_CODE_INVALID_MSG;
×
326
    goto _OVER;
×
327
  }
328

329
  if (ths->arbTerm > req.arbTerm) {
×
330
    sInfo("vgId:%d, skip to set assigned leader, msg with lower term, local:%" PRId64 "msg:%" PRId64, ths->vgId,
×
331
          ths->arbTerm, req.arbTerm);
332
    goto _OVER;
×
333
  }
334

335
  ths->arbTerm = TMAX(req.arbTerm, ths->arbTerm);
×
336

337
  if (strncmp(req.memberToken, ths->arbToken, TSDB_ARB_TOKEN_SIZE) != 0) {
×
338
    sInfo("vgId:%d, skip to set assigned leader, token mismatch, local:%s, msg:%s", ths->vgId, ths->arbToken,
×
339
          req.memberToken);
340
    goto _OVER;
×
341
  }
342

343
  if (ths->state != TAOS_SYNC_STATE_ASSIGNED_LEADER) {
×
344
    code = TSDB_CODE_SUCCESS;
×
345
    raftStoreNextTerm(ths);
×
346
    if (terrno != TSDB_CODE_SUCCESS) {
×
347
      code = terrno;
×
348
      sError("vgId:%d, failed to set next term since:%s", ths->vgId, tstrerror(code));
×
349
      goto _OVER;
×
350
    }
351
    syncNodeBecomeAssignedLeader(ths);
×
352

353
    if ((code = syncNodeAppendNoop(ths)) < 0) {
×
354
      sError("vgId:%d, assigned leader failed to append noop entry since %s", ths->vgId, tstrerror(code));
×
355
    }
356
  }
357

358
  SVArbSetAssignedLeaderRsp rsp = {0};
×
359
  rsp.arbToken = req.arbToken;
×
360
  rsp.memberToken = req.memberToken;
×
361
  rsp.vgId = ths->vgId;
×
362

363
  contLen = tSerializeSVArbSetAssignedLeaderRsp(NULL, 0, &rsp);
×
364
  if (contLen <= 0) {
×
365
    code = TSDB_CODE_OUT_OF_MEMORY;
×
366
    sError("vgId:%d, failed to serialize SVArbSetAssignedLeaderRsp", ths->vgId);
×
367
    goto _OVER;
×
368
  }
369
  pHead = rpcMallocCont(contLen);
×
370
  if (!pHead) {
×
371
    code = terrno;
×
372
    sError("vgId:%d, failed to malloc memory for SVArbSetAssignedLeaderRsp", ths->vgId);
×
373
    goto _OVER;
×
374
  }
375
  if (tSerializeSVArbSetAssignedLeaderRsp(pHead, contLen, &rsp) <= 0) {
×
376
    code = TSDB_CODE_OUT_OF_MEMORY;
×
377
    sError("vgId:%d, failed to serialize SVArbSetAssignedLeaderRsp", ths->vgId);
×
378
    rpcFreeCont(pHead);
×
379
    goto _OVER;
×
380
  }
381

382
  code = TSDB_CODE_SUCCESS;
×
383

384
_OVER:;
×
385
  SRpcMsg rspMsg = {
×
386
      .code = code,
387
      .pCont = pHead,
388
      .contLen = contLen,
389
      .info = pRpcMsg->info,
390
  };
391

392
  tmsgSendRsp(&rspMsg);
×
393

394
  tFreeSVArbSetAssignedLeaderReq(&req);
×
395
  TAOS_RETURN(code);
×
396
}
397

398
int32_t syncSendTimeoutRsp(int64_t rid, int64_t seq) {
×
399
  int32_t    code = 0;
×
400
  SSyncNode* pNode = syncNodeAcquire(rid);
×
401
  if (pNode == NULL) {
×
402
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
403
    if (terrno != 0) code = terrno;
×
404
    TAOS_RETURN(code);
×
405
  }
406

407
  SRpcMsg rpcMsg = {0, .info.notFreeAhandle = 1};
×
408
  int32_t ret = syncRespMgrGetAndDel(pNode->pSyncRespMgr, seq, &rpcMsg.info);
×
409
  rpcMsg.code = TSDB_CODE_SYN_TIMEOUT;
×
410

411
  syncNodeRelease(pNode);
×
412
  if (ret == 1) {
×
413
    sInfo("send timeout response, seq:%" PRId64 " handle:%p ahandle:%p", seq, rpcMsg.info.handle, rpcMsg.info.ahandle);
×
414
    code = rpcSendResponse(&rpcMsg);
×
415
    return code;
×
416
  } else {
417
    sError("no message handle to send timeout response, seq:%" PRId64, seq);
×
418
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
419
  }
420
}
421

422
SyncIndex syncMinMatchIndex(SSyncNode* pSyncNode) {
33,078✔
423
  SyncIndex minMatchIndex = SYNC_INDEX_INVALID;
33,078✔
424

425
  if (pSyncNode->peersNum > 0) {
33,078!
426
    minMatchIndex = syncIndexMgrGetIndex(pSyncNode->pMatchIndex, &(pSyncNode->peersId[0]));
33,078✔
427
  }
428

429
  for (int32_t i = 1; i < pSyncNode->peersNum; ++i) {
65,825✔
430
    SyncIndex matchIndex = syncIndexMgrGetIndex(pSyncNode->pMatchIndex, &(pSyncNode->peersId[i]));
32,747✔
431
    if (matchIndex < minMatchIndex) {
32,747✔
432
      minMatchIndex = matchIndex;
2,456✔
433
    }
434
  }
435
  return minMatchIndex;
33,078✔
436
}
437

438
static SyncIndex syncLogRetentionIndex(SSyncNode* pSyncNode, int64_t bytes) {
644✔
439
  return pSyncNode->pLogStore->syncLogIndexRetention(pSyncNode->pLogStore, bytes);
644✔
440
}
441

442
int32_t syncBeginSnapshot(int64_t rid, int64_t lastApplyIndex) {
16,667✔
443
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
16,667✔
444
  int32_t    code = 0;
16,667✔
445
  if (pSyncNode == NULL) {
16,667✔
446
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
12✔
447
    if (terrno != 0) code = terrno;
12!
448
    sError("sync begin snapshot error");
12!
449
    TAOS_RETURN(code);
12✔
450
  }
451

452
  SyncIndex beginIndex = pSyncNode->pLogStore->syncLogBeginIndex(pSyncNode->pLogStore);
16,655✔
453
  SyncIndex endIndex = pSyncNode->pLogStore->syncLogEndIndex(pSyncNode->pLogStore);
16,655✔
454
  bool      isEmpty = pSyncNode->pLogStore->syncLogIsEmpty(pSyncNode->pLogStore);
16,655✔
455

456
  if (isEmpty || !(lastApplyIndex >= beginIndex && lastApplyIndex <= endIndex)) {
16,655!
457
    sNTrace(pSyncNode, "new-snapshot-index:%" PRId64 ", empty:%d, do not delete wal", lastApplyIndex, isEmpty);
62!
458
    syncNodeRelease(pSyncNode);
62✔
459
    return 0;
62✔
460
  }
461

462
  int64_t logRetention = 0;
16,593✔
463

464
  if (syncNodeIsMnode(pSyncNode)) {
16,593✔
465
    // mnode
466
    logRetention = tsMndLogRetention;
2,663✔
467
  } else {
468
    // vnode
469
    if (pSyncNode->replicaNum > 1) {
13,930✔
470
      logRetention = SYNC_VNODE_LOG_RETENTION;
310✔
471
    }
472
  }
473

474
  if (pSyncNode->totalReplicaNum > 1) {
16,593✔
475
    if (pSyncNode->state != TAOS_SYNC_STATE_LEADER && pSyncNode->state != TAOS_SYNC_STATE_FOLLOWER &&
655✔
476
        pSyncNode->state != TAOS_SYNC_STATE_LEARNER && pSyncNode->state != TAOS_SYNC_STATE_ASSIGNED_LEADER) {
127!
477
      sNTrace(pSyncNode, "new-snapshot-index:%" PRId64 " candidate or unknown state, do not delete wal",
11!
478
              lastApplyIndex);
479
      syncNodeRelease(pSyncNode);
11✔
480
      return 0;
11✔
481
    }
482
    SyncIndex retentionIndex =
644✔
483
        TMAX(pSyncNode->minMatchIndex, syncLogRetentionIndex(pSyncNode, SYNC_WAL_LOG_RETENTION_SIZE));
644✔
484
    logRetention += TMAX(0, lastApplyIndex - retentionIndex);
644✔
485
  }
486

487
_DEL_WAL:
15,938✔
488

489
  do {
490
    SSyncLogStoreData* pData = pSyncNode->pLogStore->data;
16,582✔
491
    SyncIndex          snapshotVer = walGetSnapshotVer(pData->pWal);
16,582✔
492
    SyncIndex          walCommitVer = walGetCommittedVer(pData->pWal);
16,582✔
493
    SyncIndex          wallastVer = walGetLastVer(pData->pWal);
16,582✔
494
    if (lastApplyIndex <= walCommitVer) {
16,582!
495
      SyncIndex snapshottingIndex = atomic_load_64(&pSyncNode->snapshottingIndex);
16,582✔
496

497
      if (snapshottingIndex == SYNC_INDEX_INVALID) {
16,582!
498
        atomic_store_64(&pSyncNode->snapshottingIndex, lastApplyIndex);
16,582✔
499
        pSyncNode->snapshottingTime = taosGetTimestampMs();
16,582✔
500

501
        code = walBeginSnapshot(pData->pWal, lastApplyIndex, logRetention);
16,582✔
502
        if (code == 0) {
16,582!
503
          sNTrace(pSyncNode, "wal snapshot begin, index:%" PRId64 ", last apply index:%" PRId64,
16,582✔
504
                  pSyncNode->snapshottingIndex, lastApplyIndex);
505
        } else {
506
          sNError(pSyncNode, "wal snapshot begin error since:%s, index:%" PRId64 ", last apply index:%" PRId64,
×
507
                  terrstr(), pSyncNode->snapshottingIndex, lastApplyIndex);
508
          atomic_store_64(&pSyncNode->snapshottingIndex, SYNC_INDEX_INVALID);
×
509
        }
510

511
      } else {
512
        sNTrace(pSyncNode, "snapshotting for %" PRId64 ", do not delete wal for new-snapshot-index:%" PRId64,
×
513
                snapshottingIndex, lastApplyIndex);
514
      }
515
    }
516
  } while (0);
517

518
  syncNodeRelease(pSyncNode);
16,582✔
519
  TAOS_RETURN(code);
16,582✔
520
}
521

522
int32_t syncEndSnapshot(int64_t rid) {
16,655✔
523
  int32_t    code = 0;
16,655✔
524
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
16,655✔
525
  if (pSyncNode == NULL) {
16,655!
526
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
527
    if (terrno != 0) code = terrno;
×
528
    sError("sync end snapshot error");
×
529
    TAOS_RETURN(code);
×
530
  }
531

532
  if (atomic_load_64(&pSyncNode->snapshottingIndex) != SYNC_INDEX_INVALID) {
16,655✔
533
    SSyncLogStoreData* pData = pSyncNode->pLogStore->data;
16,582✔
534
    code = walEndSnapshot(pData->pWal);
16,582✔
535
    if (code != 0) {
16,582!
536
      sNError(pSyncNode, "wal snapshot end error since:%s", tstrerror(code));
×
537
      syncNodeRelease(pSyncNode);
×
538
      TAOS_RETURN(code);
×
539
    } else {
540
      sNTrace(pSyncNode, "wal snapshot end, index:%" PRId64, atomic_load_64(&pSyncNode->snapshottingIndex));
16,582✔
541
      atomic_store_64(&pSyncNode->snapshottingIndex, SYNC_INDEX_INVALID);
16,582✔
542
    }
543
  }
544

545
  syncNodeRelease(pSyncNode);
16,655✔
546
  TAOS_RETURN(code);
16,655✔
547
}
548

549
bool syncNodeIsReadyForRead(SSyncNode* pSyncNode) {
3,236,991✔
550
  if (pSyncNode == NULL) {
3,236,991!
551
    terrno = TSDB_CODE_SYN_INTERNAL_ERROR;
×
552
    sError("sync ready for read error");
×
553
    return false;
×
554
  }
555

556
  if (pSyncNode->state != TAOS_SYNC_STATE_LEADER && pSyncNode->state != TAOS_SYNC_STATE_ASSIGNED_LEADER) {
3,236,991!
557
    terrno = TSDB_CODE_SYN_NOT_LEADER;
59,199✔
558
    return false;
59,197✔
559
  }
560

561
  if (!pSyncNode->restoreFinish) {
3,177,792✔
562
    terrno = TSDB_CODE_SYN_RESTORING;
29,961✔
563
    return false;
29,961✔
564
  }
565

566
  return true;
3,147,831✔
567
}
568

569
bool syncIsReadyForRead(int64_t rid) {
2,387,301✔
570
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
2,387,301✔
571
  if (pSyncNode == NULL) {
2,388,804!
572
    sError("sync ready for read error");
×
573
    return false;
×
574
  }
575

576
  bool ready = syncNodeIsReadyForRead(pSyncNode);
2,388,804✔
577

578
  syncNodeRelease(pSyncNode);
2,388,781✔
579
  return ready;
2,388,492✔
580
}
581

582
#ifdef BUILD_NO_CALL
583
bool syncSnapshotSending(int64_t rid) {
584
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
585
  if (pSyncNode == NULL) {
586
    return false;
587
  }
588

589
  bool b = syncNodeSnapshotSending(pSyncNode);
590
  syncNodeRelease(pSyncNode);
591
  return b;
592
}
593

594
bool syncSnapshotRecving(int64_t rid) {
595
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
596
  if (pSyncNode == NULL) {
597
    return false;
598
  }
599

600
  bool b = syncNodeSnapshotRecving(pSyncNode);
601
  syncNodeRelease(pSyncNode);
602
  return b;
603
}
604
#endif
605

606
int32_t syncNodeLeaderTransfer(SSyncNode* pSyncNode) {
11,569✔
607
  if (pSyncNode->peersNum == 0) {
11,569✔
608
    sDebug("vgId:%d, only one replica, cannot leader transfer", pSyncNode->vgId);
8,899✔
609
    return 0;
8,899✔
610
  }
611

612
  int32_t ret = 0;
2,670✔
613
  if (pSyncNode->state == TAOS_SYNC_STATE_LEADER && pSyncNode->replicaNum > 1) {
2,670✔
614
    SNodeInfo newLeader = (pSyncNode->peersNodeInfo)[0];
801✔
615
    if (pSyncNode->peersNum == 2) {
801✔
616
      SyncIndex matchIndex0 = syncIndexMgrGetIndex(pSyncNode->pMatchIndex, &(pSyncNode->peersId[0]));
511✔
617
      SyncIndex matchIndex1 = syncIndexMgrGetIndex(pSyncNode->pMatchIndex, &(pSyncNode->peersId[1]));
511✔
618
      if (matchIndex1 > matchIndex0) {
511✔
619
        newLeader = (pSyncNode->peersNodeInfo)[1];
25✔
620
      }
621
    }
622
    ret = syncNodeLeaderTransferTo(pSyncNode, newLeader);
801✔
623
  }
624

625
  return ret;
2,670✔
626
}
627

628
int32_t syncNodeLeaderTransferTo(SSyncNode* pSyncNode, SNodeInfo newLeader) {
801✔
629
  if (pSyncNode->replicaNum == 1) {
801!
630
    sDebug("vgId:%d, only one replica, cannot leader transfer", pSyncNode->vgId);
×
631
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
632
  }
633

634
  sNTrace(pSyncNode, "begin leader transfer to %s:%u", newLeader.nodeFqdn, newLeader.nodePort);
801!
635

636
  SRpcMsg rpcMsg = {0};
801✔
637
  TAOS_CHECK_RETURN(syncBuildLeaderTransfer(&rpcMsg, pSyncNode->vgId));
801!
638

639
  SyncLeaderTransfer* pMsg = rpcMsg.pCont;
801✔
640
  pMsg->newLeaderId.addr = SYNC_ADDR(&newLeader);
801✔
641
  pMsg->newLeaderId.vgId = pSyncNode->vgId;
801✔
642
  pMsg->newNodeInfo = newLeader;
801✔
643

644
  int32_t ret = syncNodePropose(pSyncNode, &rpcMsg, false, NULL);
801✔
645
  rpcFreeCont(rpcMsg.pCont);
801✔
646
  return ret;
801✔
647
}
648

649
SSyncState syncGetState(int64_t rid) {
1,722,805✔
650
  SSyncState state = {.state = TAOS_SYNC_STATE_ERROR};
1,722,805✔
651

652
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
1,722,805✔
653
  if (pSyncNode != NULL) {
1,723,198✔
654
    state.state = pSyncNode->state;
1,723,191✔
655
    state.roleTimeMs = pSyncNode->roleTimeMs;
1,723,191✔
656
    state.startTimeMs = pSyncNode->startTime;
1,723,191✔
657
    state.restored = pSyncNode->restoreFinish;
1,723,191✔
658
    if (pSyncNode->vgId != 1) {
1,723,191✔
659
      state.canRead = syncNodeIsReadyForRead(pSyncNode);
849,605✔
660
    } else {
661
      state.canRead = state.restored;
873,586✔
662
    }
663
    /*
664
    double progress = 0;
665
    if(pSyncNode->pLogBuf->totalIndex > 0 && pSyncNode->pLogBuf->commitIndex > 0){
666
      progress = (double)pSyncNode->pLogBuf->commitIndex/(double)pSyncNode->pLogBuf->totalIndex;
667
      state.progress = (int32_t)(progress * 100);
668
    }
669
    else{
670
      state.progress = -1;
671
    }
672
    sDebug("vgId:%d, learner progress state, commitIndex:%" PRId64 " totalIndex:%" PRId64 ", "
673
            "progress:%lf, progress:%d",
674
          pSyncNode->vgId,
675
         pSyncNode->pLogBuf->commitIndex, pSyncNode->pLogBuf->totalIndex, progress, state.progress);
676
    */
677
    state.term = raftStoreGetTerm(pSyncNode);
1,723,157✔
678
    syncNodeRelease(pSyncNode);
1,723,179✔
679
  }
680

681
  return state;
1,723,157✔
682
}
683

684
int32_t syncGetArbToken(int64_t rid, char* outToken) {
11,840✔
685
  int32_t    code = 0;
11,840✔
686
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
11,840✔
687
  if (pSyncNode == NULL) {
11,840!
688
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
689
    if (terrno != 0) code = terrno;
×
690
    TAOS_RETURN(code);
×
691
  }
692

693
  memset(outToken, 0, TSDB_ARB_TOKEN_SIZE);
11,840✔
694
  (void)taosThreadMutexLock(&pSyncNode->arbTokenMutex);
11,840✔
695
  tstrncpy(outToken, pSyncNode->arbToken, TSDB_ARB_TOKEN_SIZE);
11,840✔
696
  (void)taosThreadMutexUnlock(&pSyncNode->arbTokenMutex);
11,840✔
697

698
  syncNodeRelease(pSyncNode);
11,840✔
699
  TAOS_RETURN(code);
11,840✔
700
}
701

702
int32_t syncGetAssignedLogSynced(int64_t rid) {
×
703
  int32_t    code = 0;
×
704
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
×
705
  if (pSyncNode == NULL) {
×
706
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
707
    if (terrno != 0) code = terrno;
×
708
    TAOS_RETURN(code);
×
709
  }
710

711
  if (pSyncNode->state != TAOS_SYNC_STATE_LEADER) {
×
712
    code = TSDB_CODE_VND_ARB_NOT_SYNCED;
×
713
    syncNodeRelease(pSyncNode);
×
714
    TAOS_RETURN(code);
×
715
  }
716

717
  bool isSync = pSyncNode->commitIndex >= pSyncNode->assignedCommitIndex;
×
718
  code = (isSync ? TSDB_CODE_SUCCESS : TSDB_CODE_VND_ARB_NOT_SYNCED);
×
719

720
  syncNodeRelease(pSyncNode);
×
721
  TAOS_RETURN(code);
×
722
}
723

724
int32_t syncUpdateArbTerm(int64_t rid, SyncTerm arbTerm) {
16✔
725
  int32_t    code = 0;
16✔
726
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
16✔
727
  if (pSyncNode == NULL) {
16!
728
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
729
    if (terrno != 0) code = terrno;
×
730
    TAOS_RETURN(code);
×
731
  }
732

733
  pSyncNode->arbTerm = TMAX(arbTerm, pSyncNode->arbTerm);
16✔
734
  syncNodeRelease(pSyncNode);
16✔
735
  TAOS_RETURN(code);
16✔
736
}
737

738
SyncIndex syncNodeGetSnapshotConfigIndex(SSyncNode* pSyncNode, SyncIndex snapshotLastApplyIndex) {
3,245,980✔
739
  if (!(pSyncNode->raftCfg.configIndexCount >= 1)) {
3,245,980!
740
    sError("vgId:%d, failed get snapshot config index, configIndexCount:%d", pSyncNode->vgId,
×
741
           pSyncNode->raftCfg.configIndexCount);
742
    terrno = TSDB_CODE_SYN_INTERNAL_ERROR;
×
743
    return -2;
×
744
  }
745
  SyncIndex lastIndex = (pSyncNode->raftCfg.configIndexArr)[0];
3,245,980✔
746

747
  for (int32_t i = 0; i < pSyncNode->raftCfg.configIndexCount; ++i) {
6,536,113✔
748
    if ((pSyncNode->raftCfg.configIndexArr)[i] > lastIndex &&
3,290,133✔
749
        (pSyncNode->raftCfg.configIndexArr)[i] <= snapshotLastApplyIndex) {
44,155!
750
      lastIndex = (pSyncNode->raftCfg.configIndexArr)[i];
44,155✔
751
    }
752
  }
753
  sTrace("vgId:%d, sync get last config index, index:%" PRId64 " lcindex:%" PRId64, pSyncNode->vgId,
3,245,980✔
754
         snapshotLastApplyIndex, lastIndex);
755

756
  return lastIndex;
3,245,980✔
757
}
758

759
static SRaftId syncGetRaftIdByEp(SSyncNode* pSyncNode, const SEp* pEp) {
92,202✔
760
  for (int32_t i = 0; i < pSyncNode->peersNum; ++i) {
187,684✔
761
    if (strcmp(pEp->fqdn, (pSyncNode->peersNodeInfo)[i].nodeFqdn) == 0 &&
155,870!
762
        pEp->port == (pSyncNode->peersNodeInfo)[i].nodePort) {
155,870✔
763
      return pSyncNode->peersId[i];
60,388✔
764
    }
765
  }
766
  return EMPTY_RAFT_ID;
31,814✔
767
}
768

769
void syncGetRetryEpSet(int64_t rid, SEpSet* pEpSet) {
31,818✔
770
  pEpSet->numOfEps = 0;
31,818✔
771

772
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
31,818✔
773
  if (pSyncNode == NULL) return;
31,818!
774

775
  int index = -1;
31,818✔
776

777
  int j = 0;
31,818✔
778
  for (int32_t i = 0; i < pSyncNode->raftCfg.cfg.totalReplicaNum; ++i) {
125,094✔
779
    if (pSyncNode->raftCfg.cfg.nodeInfo[i].nodeRole == TAOS_SYNC_ROLE_LEARNER) continue;
93,276✔
780
    SEp* pEp = &pEpSet->eps[j];
92,202✔
781
    tstrncpy(pEp->fqdn, pSyncNode->raftCfg.cfg.nodeInfo[i].nodeFqdn, TSDB_FQDN_LEN);
92,202✔
782
    pEp->port = (pSyncNode->raftCfg.cfg.nodeInfo)[i].nodePort;
92,202✔
783
    pEpSet->numOfEps++;
92,202✔
784
    sDebug("vgId:%d, sync get retry epset, index:%d %s:%d", pSyncNode->vgId, i, pEp->fqdn, pEp->port);
92,202✔
785
    SRaftId id = syncGetRaftIdByEp(pSyncNode, pEp);
92,202✔
786
    if (id.addr == pSyncNode->leaderCache.addr && id.vgId == pSyncNode->leaderCache.vgId && id.addr != 0 &&
92,202!
787
        id.vgId != 0)
11,020!
788
      index = j;
11,020✔
789
    j++;
92,202✔
790
  }
791
  if (pEpSet->numOfEps > 0) {
31,818!
792
    if (index != -1) {
31,818✔
793
      pEpSet->inUse = index;
11,020✔
794
    } else {
795
      if (pSyncNode->myRaftId.addr == pSyncNode->leaderCache.addr &&
20,798✔
796
          pSyncNode->myRaftId.vgId == pSyncNode->leaderCache.vgId) {
693!
797
        pEpSet->inUse = pSyncNode->raftCfg.cfg.myIndex;
693✔
798
      } else {
799
        pEpSet->inUse = (pSyncNode->raftCfg.cfg.myIndex + 1) % pEpSet->numOfEps;
20,105✔
800
      }
801
    }
802
    // pEpSet->inUse = 0;
803
  }
804
  epsetSort(pEpSet);
31,818✔
805

806
  sInfo("vgId:%d, sync get retry epset numOfEps:%d inUse:%d", pSyncNode->vgId, pEpSet->numOfEps, pEpSet->inUse);
31,818!
807
  syncNodeRelease(pSyncNode);
31,818✔
808
}
809

810
int32_t syncPropose(int64_t rid, SRpcMsg* pMsg, bool isWeak, int64_t* seq) {
9,701,507✔
811
  int32_t    code = 0;
9,701,507✔
812
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
9,701,507✔
813
  if (pSyncNode == NULL) {
9,701,665✔
814
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
1✔
815
    if (terrno != 0) code = terrno;
1!
816
    sError("sync propose error");
1!
817
    TAOS_RETURN(code);
1✔
818
  }
819

820
  int32_t ret = syncNodePropose(pSyncNode, pMsg, isWeak, seq);
9,701,664✔
821
  syncNodeRelease(pSyncNode);
9,701,321✔
822
  return ret;
9,701,514✔
823
}
824

825
int32_t syncCheckMember(int64_t rid) {
×
826
  int32_t    code = 0;
×
827
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
×
828
  if (pSyncNode == NULL) {
×
829
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
830
    if (terrno != 0) code = terrno;
×
831
    sError("sync propose error");
×
832
    TAOS_RETURN(code);
×
833
  }
834

835
  if (pSyncNode->myNodeInfo.nodeRole == TAOS_SYNC_ROLE_LEARNER) {
×
836
    syncNodeRelease(pSyncNode);
×
837
    return TSDB_CODE_SYN_WRONG_ROLE;
×
838
  }
839

840
  syncNodeRelease(pSyncNode);
×
841
  return 0;
×
842
}
843

844
int32_t syncIsCatchUp(int64_t rid) {
5,152✔
845
  int32_t    code = 0;
5,152✔
846
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
5,152✔
847
  if (pSyncNode == NULL) {
5,152!
848
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
849
    if (terrno != 0) code = terrno;
×
850
    sError("sync Node Acquire error since %d", errno);
×
851
    TAOS_RETURN(code);
×
852
  }
853

854
  int32_t isCatchUp = 0;
5,152✔
855
  if (pSyncNode->pLogBuf->totalIndex < 0 || pSyncNode->pLogBuf->commitIndex < 0 ||
5,152!
856
      pSyncNode->pLogBuf->totalIndex < pSyncNode->pLogBuf->commitIndex ||
1,299!
857
      pSyncNode->pLogBuf->totalIndex - pSyncNode->pLogBuf->commitIndex > SYNC_LEARNER_CATCHUP) {
1,299✔
858
    sInfo("vgId:%d, Not catch up, wait one second, totalIndex:%" PRId64 " commitIndex:%" PRId64 " matchIndex:%" PRId64,
4,910!
859
          pSyncNode->vgId, pSyncNode->pLogBuf->totalIndex, pSyncNode->pLogBuf->commitIndex,
860
          pSyncNode->pLogBuf->matchIndex);
861
    isCatchUp = 0;
4,910✔
862
  } else {
863
    sInfo("vgId:%d, Catch up, totalIndex:%" PRId64 " commitIndex:%" PRId64 " matchIndex:%" PRId64, pSyncNode->vgId,
242!
864
          pSyncNode->pLogBuf->totalIndex, pSyncNode->pLogBuf->commitIndex, pSyncNode->pLogBuf->matchIndex);
865
    isCatchUp = 1;
242✔
866
  }
867

868
  syncNodeRelease(pSyncNode);
5,152✔
869
  return isCatchUp;
5,152✔
870
}
871

872
ESyncRole syncGetRole(int64_t rid) {
5,152✔
873
  int32_t    code = 0;
5,152✔
874
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
5,152✔
875
  if (pSyncNode == NULL) {
5,152!
876
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
877
    if (terrno != 0) code = terrno;
×
878
    sError("sync Node Acquire error since %d", errno);
×
879
    TAOS_RETURN(code);
×
880
  }
881

882
  ESyncRole role = pSyncNode->raftCfg.cfg.nodeInfo[pSyncNode->raftCfg.cfg.myIndex].nodeRole;
5,152✔
883

884
  syncNodeRelease(pSyncNode);
5,152✔
885
  return role;
5,152✔
886
}
887

888
int64_t syncGetTerm(int64_t rid) {
3,798✔
889
  int32_t    code = 0;
3,798✔
890
  SSyncNode* pSyncNode = syncNodeAcquire(rid);
3,798✔
891
  if (pSyncNode == NULL) {
3,798!
892
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
893
    if (terrno != 0) code = terrno;
×
894
    sError("sync Node Acquire error since %d", errno);
×
895
    TAOS_RETURN(code);
×
896
  }
897

898
  int64_t term = raftStoreGetTerm(pSyncNode);
3,798✔
899

900
  syncNodeRelease(pSyncNode);
3,798✔
901
  return term;
3,798✔
902
}
903

904
int32_t syncNodePropose(SSyncNode* pSyncNode, SRpcMsg* pMsg, bool isWeak, int64_t* seq) {
9,702,220✔
905
  int32_t code = 0;
9,702,220✔
906
  if (pSyncNode->state != TAOS_SYNC_STATE_LEADER && pSyncNode->state != TAOS_SYNC_STATE_ASSIGNED_LEADER) {
9,702,220!
907
    code = TSDB_CODE_SYN_NOT_LEADER;
7,146✔
908
    sNWarn(pSyncNode, "sync propose not leader, type:%s", TMSG_INFO(pMsg->msgType));
7,146!
909
    TAOS_RETURN(code);
7,146✔
910
  }
911

912
  if (!pSyncNode->restoreFinish) {
9,695,074!
UNCOV
913
    code = TSDB_CODE_SYN_PROPOSE_NOT_READY;
×
UNCOV
914
    sNWarn(pSyncNode, "failed to sync propose since not ready, type:%s, last:%" PRId64 ", cmt:%" PRId64,
×
915
           TMSG_INFO(pMsg->msgType), syncNodeGetLastIndex(pSyncNode), pSyncNode->commitIndex);
UNCOV
916
    TAOS_RETURN(code);
×
917
  }
918

919
  // heartbeat timeout
920
  if (pSyncNode->state != TAOS_SYNC_STATE_ASSIGNED_LEADER && syncNodeHeartbeatReplyTimeout(pSyncNode)) {
9,695,074!
921
    code = TSDB_CODE_SYN_PROPOSE_NOT_READY;
×
922
    sNError(pSyncNode, "failed to sync propose since heartbeat timeout, type:%s, last:%" PRId64 ", cmt:%" PRId64,
×
923
            TMSG_INFO(pMsg->msgType), syncNodeGetLastIndex(pSyncNode), pSyncNode->commitIndex);
924
    TAOS_RETURN(code);
×
925
  }
926

927
  // optimized one replica
928
  if (syncNodeIsOptimizedOneReplica(pSyncNode, pMsg)) {
9,695,172✔
929
    SyncIndex retIndex;
930
    int32_t   code = syncNodeOnClientRequest(pSyncNode, pMsg, &retIndex);
9,488,875✔
931
    if (code >= 0) {
9,488,534!
932
      pMsg->info.conn.applyIndex = retIndex;
9,488,612✔
933
      pMsg->info.conn.applyTerm = raftStoreGetTerm(pSyncNode);
9,488,612✔
934

935
      // after raft member change, need to handle 1->2 switching point
936
      // at this point, need to switch entry handling thread
937
      if (pSyncNode->replicaNum == 1) {
9,488,915✔
938
        sTrace("vgId:%d, propose optimized msg, index:%" PRId64 " type:%s", pSyncNode->vgId, retIndex,
9,488,893!
939
               TMSG_INFO(pMsg->msgType));
940
        return 1;
9,488,867✔
941
      } else {
942
        sTrace("vgId:%d, propose optimized msg, return to normal, index:%" PRId64
22!
943
               " type:%s, "
944
               "handle:%p",
945
               pSyncNode->vgId, retIndex, TMSG_INFO(pMsg->msgType), pMsg->info.handle);
946
        return 0;
×
947
      }
948
    } else {
949
      code = TSDB_CODE_SYN_INTERNAL_ERROR;
×
950
      sError("vgId:%d, failed to propose optimized msg, index:%" PRId64 " type:%s", pSyncNode->vgId, retIndex,
×
951
             TMSG_INFO(pMsg->msgType));
952
      TAOS_RETURN(code);
×
953
    }
954
  } else {
955
    SRespStub stub = {.createTime = taosGetTimestampMs(), .rpcMsg = *pMsg};
206,385✔
956
    uint64_t  seqNum = syncRespMgrAdd(pSyncNode->pSyncRespMgr, &stub);
206,397✔
957
    SRpcMsg   rpcMsg = {0};
206,401✔
958
    int32_t   code = syncBuildClientRequest(&rpcMsg, pMsg, seqNum, isWeak, pSyncNode->vgId);
206,401✔
959
    if (code != 0) {
206,402!
960
      sError("vgId:%d, failed to propose msg while serialize since %s", pSyncNode->vgId, terrstr());
×
961
      code = syncRespMgrDel(pSyncNode->pSyncRespMgr, seqNum);
×
962
      TAOS_RETURN(code);
×
963
    }
964

965
    sNTrace(pSyncNode, "propose msg, type:%s", TMSG_INFO(pMsg->msgType));
206,402!
966
    code = (*pSyncNode->syncEqMsg)(pSyncNode->msgcb, &rpcMsg);
206,402✔
967
    if (code != 0) {
206,399✔
968
      sWarn("vgId:%d, failed to propose msg while enqueue since %s", pSyncNode->vgId, terrstr());
1,138!
969
      TAOS_CHECK_RETURN(syncRespMgrDel(pSyncNode->pSyncRespMgr, seqNum));
1,138✔
970
    }
971

972
    if (seq != NULL) *seq = seqNum;
206,354✔
973
    TAOS_RETURN(code);
206,354✔
974
  }
975
}
976

977
static int32_t syncHbTimerInit(SSyncNode* pSyncNode, SSyncTimer* pSyncTimer, SRaftId destId) {
192,705✔
978
  pSyncTimer->pTimer = NULL;
192,705✔
979
  pSyncTimer->counter = 0;
192,705✔
980
  pSyncTimer->timerMS = pSyncNode->hbBaseLine;
192,705✔
981
  pSyncTimer->timerCb = syncNodeEqPeerHeartbeatTimer;
192,705✔
982
  pSyncTimer->destId = destId;
192,705✔
983
  pSyncTimer->timeStamp = taosGetTimestampMs();
192,705✔
984
  atomic_store_64(&pSyncTimer->logicClock, 0);
192,705✔
985
  return 0;
192,705✔
986
}
987

988
static int32_t syncHbTimerStart(SSyncNode* pSyncNode, SSyncTimer* pSyncTimer) {
1,893✔
989
  int32_t code = 0;
1,893✔
990
  int64_t tsNow = taosGetTimestampMs();
1,893✔
991
  if (syncIsInit()) {
1,893!
992
    SSyncHbTimerData* pData = syncHbTimerDataAcquire(pSyncTimer->hbDataRid);
1,893✔
993
    if (pData == NULL) {
1,893!
994
      pData = taosMemoryMalloc(sizeof(SSyncHbTimerData));
1,893!
995
      pData->rid = syncHbTimerDataAdd(pData);
1,893✔
996
    }
997
    pSyncTimer->hbDataRid = pData->rid;
1,893✔
998
    pSyncTimer->timeStamp = tsNow;
1,893✔
999

1000
    pData->syncNodeRid = pSyncNode->rid;
1,893✔
1001
    pData->pTimer = pSyncTimer;
1,893✔
1002
    pData->destId = pSyncTimer->destId;
1,893✔
1003
    pData->logicClock = pSyncTimer->logicClock;
1,893✔
1004
    pData->execTime = tsNow + pSyncTimer->timerMS;
1,893✔
1005

1006
    sTrace("vgId:%d, start hb timer, rid:%" PRId64 " addr:%" PRId64 " at %d", pSyncNode->vgId, pData->rid,
1,893!
1007
           pData->destId.addr, pSyncTimer->timerMS);
1008

1009
    bool stopped = taosTmrReset(pSyncTimer->timerCb, pSyncTimer->timerMS, (void*)(pData->rid), syncEnv()->pTimerManager,
1,893✔
1010
                                &pSyncTimer->pTimer);
1,893✔
1011
    if (stopped) {
1,893!
1012
      sError("vgId:%d, failed to reset hb timer success", pSyncNode->vgId);
×
1013
      return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1014
    }
1015
  } else {
1016
    code = TSDB_CODE_SYN_INTERNAL_ERROR;
×
1017
    sError("vgId:%d, start ctrl hb timer error, sync env is stop", pSyncNode->vgId);
×
1018
  }
1019
  return code;
1,893✔
1020
}
1021

1022
static int32_t syncHbTimerStop(SSyncNode* pSyncNode, SSyncTimer* pSyncTimer) {
19,616✔
1023
  int32_t ret = 0;
19,616✔
1024
  (void)atomic_add_fetch_64(&pSyncTimer->logicClock, 1);
19,616✔
1025
  bool stop = taosTmrStop(pSyncTimer->pTimer);
19,616✔
1026
  sDebug("vgId:%d, stop hb timer stop:%d", pSyncNode->vgId, stop);
19,616✔
1027
  pSyncTimer->pTimer = NULL;
19,616✔
1028
  syncHbTimerDataRemove(pSyncTimer->hbDataRid);
19,616✔
1029
  pSyncTimer->hbDataRid = -1;
19,616✔
1030
  return ret;
19,616✔
1031
}
1032

1033
int32_t syncNodeLogStoreRestoreOnNeed(SSyncNode* pNode) {
11,570✔
1034
  int32_t code = 0;
11,570✔
1035
  if (pNode->pLogStore == NULL) {
11,570!
1036
    sError("vgId:%d, log store not created", pNode->vgId);
×
1037
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1038
  }
1039
  if (pNode->pFsm == NULL) {
11,570!
1040
    sError("vgId:%d, pFsm not registered", pNode->vgId);
×
1041
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1042
  }
1043
  if (pNode->pFsm->FpGetSnapshotInfo == NULL) {
11,570!
1044
    sError("vgId:%d, FpGetSnapshotInfo not registered", pNode->vgId);
×
1045
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1046
  }
1047
  SSnapshot snapshot = {0};
11,570✔
1048
  // TODO check return value
1049
  (void)pNode->pFsm->FpGetSnapshotInfo(pNode->pFsm, &snapshot);
11,570✔
1050

1051
  SyncIndex commitIndex = snapshot.lastApplyIndex;
11,570✔
1052
  SyncIndex firstVer = pNode->pLogStore->syncLogBeginIndex(pNode->pLogStore);
11,570✔
1053
  SyncIndex lastVer = pNode->pLogStore->syncLogLastIndex(pNode->pLogStore);
11,570✔
1054
  if ((lastVer < commitIndex || firstVer > commitIndex + 1) || pNode->fsmState == SYNC_FSM_STATE_INCOMPLETE) {
11,570!
1055
    if ((code = pNode->pLogStore->syncLogRestoreFromSnapshot(pNode->pLogStore, commitIndex)) != 0) {
×
1056
      sError("vgId:%d, failed to restore log store from snapshot since %s. lastVer:%" PRId64 ", snapshotVer:%" PRId64,
×
1057
             pNode->vgId, terrstr(), lastVer, commitIndex);
1058
      TAOS_RETURN(code);
×
1059
    }
1060
  }
1061
  TAOS_RETURN(code);
11,570✔
1062
}
1063

1064
// open/close --------------
1065
SSyncNode* syncNodeOpen(SSyncInfo* pSyncInfo, int32_t vnodeVersion) {
11,569✔
1066
  int32_t    code = 0;
11,569✔
1067
  SSyncNode* pSyncNode = taosMemoryCalloc(1, sizeof(SSyncNode));
11,569!
1068
  if (pSyncNode == NULL) {
11,569!
1069
    terrno = TSDB_CODE_OUT_OF_MEMORY;
×
1070
    goto _error;
×
1071
  }
1072

1073
  if (!taosDirExist((char*)(pSyncInfo->path))) {
11,569✔
1074
    if (taosMkDir(pSyncInfo->path) != 0) {
9,249!
1075
      terrno = TAOS_SYSTEM_ERROR(errno);
×
1076
      sError("vgId:%d, failed to create dir:%s since %s", pSyncInfo->vgId, pSyncInfo->path, terrstr());
×
1077
      goto _error;
×
1078
    }
1079
  }
1080

1081
  memcpy(pSyncNode->path, pSyncInfo->path, sizeof(pSyncNode->path));
11,569✔
1082
  snprintf(pSyncNode->raftStorePath, sizeof(pSyncNode->raftStorePath), "%s%sraft_store.json", pSyncInfo->path,
11,569✔
1083
           TD_DIRSEP);
1084
  snprintf(pSyncNode->configPath, sizeof(pSyncNode->configPath), "%s%sraft_config.json", pSyncInfo->path, TD_DIRSEP);
11,569✔
1085

1086
  if (!taosCheckExistFile(pSyncNode->configPath)) {
11,569✔
1087
    // create a new raft config file
1088
    sInfo("vgId:%d, create a new raft config file", pSyncInfo->vgId);
9,249✔
1089
    pSyncNode->vgId = pSyncInfo->vgId;
9,249✔
1090
    pSyncNode->raftCfg.isStandBy = pSyncInfo->isStandBy;
9,249✔
1091
    pSyncNode->raftCfg.snapshotStrategy = pSyncInfo->snapshotStrategy;
9,249✔
1092
    pSyncNode->raftCfg.lastConfigIndex = pSyncInfo->syncCfg.lastIndex;
9,249✔
1093
    pSyncNode->raftCfg.batchSize = pSyncInfo->batchSize;
9,249✔
1094
    pSyncNode->raftCfg.cfg = pSyncInfo->syncCfg;
9,249✔
1095
    pSyncNode->raftCfg.configIndexCount = 1;
9,249✔
1096
    pSyncNode->raftCfg.configIndexArr[0] = -1;
9,249✔
1097

1098
    if ((code = syncWriteCfgFile(pSyncNode)) != 0) {
9,249!
1099
      terrno = code;
×
1100
      sError("vgId:%d, failed to create sync cfg file", pSyncNode->vgId);
×
1101
      goto _error;
×
1102
    }
1103
  } else {
1104
    // update syncCfg by raft_config.json
1105
    if ((code = syncReadCfgFile(pSyncNode)) != 0) {
2,321!
1106
      terrno = code;
×
1107
      sError("vgId:%d, failed to read sync cfg file", pSyncNode->vgId);
×
1108
      goto _error;
×
1109
    }
1110

1111
    if (vnodeVersion > pSyncNode->raftCfg.cfg.changeVersion) {
2,321✔
1112
      if (pSyncInfo->syncCfg.totalReplicaNum > 0 && syncIsConfigChanged(&pSyncNode->raftCfg.cfg, &pSyncInfo->syncCfg)) {
1,637!
1113
        sInfo("vgId:%d, use sync config from input options and write to cfg file", pSyncNode->vgId);
1,389!
1114
        pSyncNode->raftCfg.cfg = pSyncInfo->syncCfg;
1,389✔
1115
        if ((code = syncWriteCfgFile(pSyncNode)) != 0) {
1,389!
1116
          terrno = code;
×
1117
          sError("vgId:%d, failed to write sync cfg file", pSyncNode->vgId);
×
1118
          goto _error;
×
1119
        }
1120
      } else {
1121
        sInfo("vgId:%d, use sync config from sync cfg file", pSyncNode->vgId);
248!
1122
        pSyncInfo->syncCfg = pSyncNode->raftCfg.cfg;
248✔
1123
      }
1124
    } else {
1125
      sInfo("vgId:%d, skip save sync cfg file since request ver:%d <= file ver:%d", pSyncNode->vgId, vnodeVersion,
684!
1126
            pSyncInfo->syncCfg.changeVersion);
1127
    }
1128
  }
1129

1130
  // init by SSyncInfo
1131
  pSyncNode->vgId = pSyncInfo->vgId;
11,570✔
1132
  SSyncCfg* pCfg = &pSyncNode->raftCfg.cfg;
11,570✔
1133
  bool      updated = false;
11,570✔
1134
  sInfo("vgId:%d, start to open sync node, totalReplicaNum:%d replicaNum:%d selfIndex:%d", pSyncNode->vgId,
11,570✔
1135
        pCfg->totalReplicaNum, pCfg->replicaNum, pCfg->myIndex);
1136
  for (int32_t i = 0; i < pCfg->totalReplicaNum; ++i) {
28,546✔
1137
    SNodeInfo* pNode = &pCfg->nodeInfo[i];
16,976✔
1138
    if (tmsgUpdateDnodeInfo(&pNode->nodeId, &pNode->clusterId, pNode->nodeFqdn, &pNode->nodePort)) {
16,976!
1139
      updated = true;
×
1140
    }
1141
    sInfo("vgId:%d, index:%d ep:%s:%u dnode:%d cluster:%" PRId64, pSyncNode->vgId, i, pNode->nodeFqdn, pNode->nodePort,
16,976✔
1142
          pNode->nodeId, pNode->clusterId);
1143
  }
1144

1145
  if (vnodeVersion > pSyncInfo->syncCfg.changeVersion) {
11,570✔
1146
    if (updated) {
1,518!
1147
      sInfo("vgId:%d, save config info since dnode info changed", pSyncNode->vgId);
×
1148
      if ((code = syncWriteCfgFile(pSyncNode)) != 0) {
×
1149
        terrno = code;
×
1150
        sError("vgId:%d, failed to write sync cfg file on dnode info updated", pSyncNode->vgId);
×
1151
        goto _error;
×
1152
      }
1153
    }
1154
  }
1155

1156
  pSyncNode->pWal = pSyncInfo->pWal;
11,570✔
1157
  pSyncNode->msgcb = pSyncInfo->msgcb;
11,570✔
1158
  pSyncNode->syncSendMSg = pSyncInfo->syncSendMSg;
11,570✔
1159
  pSyncNode->syncEqMsg = pSyncInfo->syncEqMsg;
11,570✔
1160
  pSyncNode->syncEqCtrlMsg = pSyncInfo->syncEqCtrlMsg;
11,570✔
1161

1162
  // create raft log ring buffer
1163
  code = syncLogBufferCreate(&pSyncNode->pLogBuf);
11,570✔
1164
  if (pSyncNode->pLogBuf == NULL) {
11,567!
1165
    sError("failed to init sync log buffer since %s. vgId:%d", tstrerror(code), pSyncNode->vgId);
×
1166
    goto _error;
×
1167
  }
1168

1169
  // init replicaNum, replicasId
1170
  pSyncNode->replicaNum = pSyncNode->raftCfg.cfg.replicaNum;
11,570✔
1171
  pSyncNode->totalReplicaNum = pSyncNode->raftCfg.cfg.totalReplicaNum;
11,570✔
1172
  for (int32_t i = 0; i < pSyncNode->raftCfg.cfg.totalReplicaNum; ++i) {
28,546✔
1173
    if (syncUtilNodeInfo2RaftId(&pSyncNode->raftCfg.cfg.nodeInfo[i], pSyncNode->vgId, &pSyncNode->replicasId[i]) ==
16,976!
1174
        false) {
1175
      sError("vgId:%d, failed to determine raft member id, replica:%d", pSyncNode->vgId, i);
×
1176
      goto _error;
×
1177
    }
1178
  }
1179

1180
  // init internal
1181
  pSyncNode->myNodeInfo = pSyncNode->raftCfg.cfg.nodeInfo[pSyncNode->raftCfg.cfg.myIndex];
11,570✔
1182
  pSyncNode->myRaftId = pSyncNode->replicasId[pSyncNode->raftCfg.cfg.myIndex];
11,570✔
1183

1184
  // init peersNum, peers, peersId
1185
  pSyncNode->peersNum = pSyncNode->raftCfg.cfg.totalReplicaNum - 1;
11,570✔
1186
  int32_t j = 0;
11,570✔
1187
  for (int32_t i = 0; i < pSyncNode->raftCfg.cfg.totalReplicaNum; ++i) {
28,546✔
1188
    if (i != pSyncNode->raftCfg.cfg.myIndex) {
16,976✔
1189
      pSyncNode->peersNodeInfo[j] = pSyncNode->raftCfg.cfg.nodeInfo[i];
5,406✔
1190
      pSyncNode->peersId[j] = pSyncNode->replicasId[i];
5,406✔
1191
      syncUtilNodeInfo2EpSet(&pSyncNode->peersNodeInfo[j], &pSyncNode->peersEpset[j]);
5,406✔
1192
      j++;
5,406✔
1193
    }
1194
  }
1195

1196
  pSyncNode->arbTerm = -1;
11,570✔
1197
  (void)taosThreadMutexInit(&pSyncNode->arbTokenMutex, NULL);
11,570✔
1198
  syncUtilGenerateArbToken(pSyncNode->myNodeInfo.nodeId, pSyncInfo->vgId, pSyncNode->arbToken);
11,570✔
1199
  sInfo("vgId:%d, generate arb token:%s", pSyncNode->vgId, pSyncNode->arbToken);
11,570✔
1200

1201
  // init raft algorithm
1202
  pSyncNode->pFsm = pSyncInfo->pFsm;
11,570✔
1203
  pSyncInfo->pFsm = NULL;
11,570✔
1204
  pSyncNode->quorum = syncUtilQuorum(pSyncNode->raftCfg.cfg.replicaNum);
11,570✔
1205
  pSyncNode->leaderCache = EMPTY_RAFT_ID;
11,570✔
1206

1207
  // init life cycle outside
1208

1209
  // TLA+ Spec
1210
  // InitHistoryVars == /\ elections = {}
1211
  //                    /\ allLogs   = {}
1212
  //                    /\ voterLog  = [i \in Server |-> [j \in {} |-> <<>>]]
1213
  // InitServerVars == /\ currentTerm = [i \in Server |-> 1]
1214
  //                   /\ state       = [i \in Server |-> Follower]
1215
  //                   /\ votedFor    = [i \in Server |-> Nil]
1216
  // InitCandidateVars == /\ votesResponded = [i \in Server |-> {}]
1217
  //                      /\ votesGranted   = [i \in Server |-> {}]
1218
  // \* The values nextIndex[i][i] and matchIndex[i][i] are never read, since the
1219
  // \* leader does not send itself messages. It's still easier to include these
1220
  // \* in the functions.
1221
  // InitLeaderVars == /\ nextIndex  = [i \in Server |-> [j \in Server |-> 1]]
1222
  //                   /\ matchIndex = [i \in Server |-> [j \in Server |-> 0]]
1223
  // InitLogVars == /\ log          = [i \in Server |-> << >>]
1224
  //                /\ commitIndex  = [i \in Server |-> 0]
1225
  // Init == /\ messages = [m \in {} |-> 0]
1226
  //         /\ InitHistoryVars
1227
  //         /\ InitServerVars
1228
  //         /\ InitCandidateVars
1229
  //         /\ InitLeaderVars
1230
  //         /\ InitLogVars
1231
  //
1232

1233
  // init TLA+ server vars
1234
  pSyncNode->state = TAOS_SYNC_STATE_FOLLOWER;
11,570✔
1235
  pSyncNode->roleTimeMs = taosGetTimestampMs();
11,570✔
1236
  if ((code = raftStoreOpen(pSyncNode)) != 0) {
11,570!
1237
    terrno = code;
×
1238
    sError("vgId:%d, failed to open raft store at path %s", pSyncNode->vgId, pSyncNode->raftStorePath);
×
1239
    goto _error;
×
1240
  }
1241

1242
  // init TLA+ candidate vars
1243
  pSyncNode->pVotesGranted = voteGrantedCreate(pSyncNode);
11,570✔
1244
  if (pSyncNode->pVotesGranted == NULL) {
11,570!
1245
    sError("vgId:%d, failed to create VotesGranted", pSyncNode->vgId);
×
1246
    goto _error;
×
1247
  }
1248
  pSyncNode->pVotesRespond = votesRespondCreate(pSyncNode);
11,570✔
1249
  if (pSyncNode->pVotesRespond == NULL) {
11,570!
1250
    sError("vgId:%d, failed to create VotesRespond", pSyncNode->vgId);
×
1251
    goto _error;
×
1252
  }
1253

1254
  // init TLA+ leader vars
1255
  pSyncNode->pNextIndex = syncIndexMgrCreate(pSyncNode);
11,570✔
1256
  if (pSyncNode->pNextIndex == NULL) {
11,570!
1257
    sError("vgId:%d, failed to create SyncIndexMgr", pSyncNode->vgId);
×
1258
    goto _error;
×
1259
  }
1260
  pSyncNode->pMatchIndex = syncIndexMgrCreate(pSyncNode);
11,570✔
1261
  if (pSyncNode->pMatchIndex == NULL) {
11,570!
1262
    sError("vgId:%d, failed to create SyncIndexMgr", pSyncNode->vgId);
×
1263
    goto _error;
×
1264
  }
1265

1266
  // init TLA+ log vars
1267
  pSyncNode->pLogStore = logStoreCreate(pSyncNode);
11,570✔
1268
  if (pSyncNode->pLogStore == NULL) {
11,570!
1269
    sError("vgId:%d, failed to create SyncLogStore", pSyncNode->vgId);
×
1270
    goto _error;
×
1271
  }
1272

1273
  SyncIndex commitIndex = SYNC_INDEX_INVALID;
11,570✔
1274
  if (pSyncNode->pFsm != NULL && pSyncNode->pFsm->FpGetSnapshotInfo != NULL) {
11,570!
1275
    SSnapshot snapshot = {0};
11,570✔
1276
    // TODO check return value
1277
    (void)pSyncNode->pFsm->FpGetSnapshotInfo(pSyncNode->pFsm, &snapshot);
11,570✔
1278
    if (snapshot.lastApplyIndex > commitIndex) {
11,570✔
1279
      commitIndex = snapshot.lastApplyIndex;
1,184✔
1280
      sNTrace(pSyncNode, "reset commit index by snapshot");
1,184!
1281
    }
1282
    pSyncNode->fsmState = snapshot.state;
11,570✔
1283
    if (pSyncNode->fsmState == SYNC_FSM_STATE_INCOMPLETE) {
11,570!
1284
      sError("vgId:%d, fsm state is incomplete.", pSyncNode->vgId);
×
1285
      if (pSyncNode->replicaNum == 1) {
×
1286
        terrno = TSDB_CODE_SYN_INTERNAL_ERROR;
×
1287
        goto _error;
×
1288
      }
1289
    }
1290
  }
1291
  pSyncNode->commitIndex = commitIndex;
11,570✔
1292
  sInfo("vgId:%d, sync node commitIndex initialized as %" PRId64, pSyncNode->vgId, pSyncNode->commitIndex);
11,570✔
1293

1294
  // restore log store on need
1295
  if ((code = syncNodeLogStoreRestoreOnNeed(pSyncNode)) < 0) {
11,570!
1296
    terrno = code;
×
1297
    sError("vgId:%d, failed to restore log store since %s.", pSyncNode->vgId, terrstr());
×
1298
    goto _error;
×
1299
  }
1300

1301
  // timer ms init
1302
  pSyncNode->pingBaseLine = PING_TIMER_MS;
11,570✔
1303
  pSyncNode->electBaseLine = tsElectInterval;
11,570✔
1304
  pSyncNode->hbBaseLine = tsHeartbeatInterval;
11,570✔
1305

1306
  // init ping timer
1307
  pSyncNode->pPingTimer = NULL;
11,570✔
1308
  pSyncNode->pingTimerMS = pSyncNode->pingBaseLine;
11,570✔
1309
  atomic_store_64(&pSyncNode->pingTimerLogicClock, 0);
11,570✔
1310
  atomic_store_64(&pSyncNode->pingTimerLogicClockUser, 0);
11,570✔
1311
  pSyncNode->FpPingTimerCB = syncNodeEqPingTimer;
11,570✔
1312
  pSyncNode->pingTimerCounter = 0;
11,570✔
1313

1314
  // init elect timer
1315
  pSyncNode->pElectTimer = NULL;
11,570✔
1316
  pSyncNode->electTimerMS = syncUtilElectRandomMS(pSyncNode->electBaseLine, 2 * pSyncNode->electBaseLine);
11,570✔
1317
  atomic_store_64(&pSyncNode->electTimerLogicClock, 0);
11,570✔
1318
  pSyncNode->FpElectTimerCB = syncNodeEqElectTimer;
11,570✔
1319
  pSyncNode->electTimerCounter = 0;
11,570✔
1320

1321
  // init heartbeat timer
1322
  pSyncNode->pHeartbeatTimer = NULL;
11,570✔
1323
  pSyncNode->heartbeatTimerMS = pSyncNode->hbBaseLine;
11,570✔
1324
  atomic_store_64(&pSyncNode->heartbeatTimerLogicClock, 0);
11,570✔
1325
  atomic_store_64(&pSyncNode->heartbeatTimerLogicClockUser, 0);
11,570✔
1326
#ifdef BUILD_NO_CALL
1327
  pSyncNode->FpHeartbeatTimerCB = syncNodeEqHeartbeatTimer;
1328
#endif
1329
  pSyncNode->heartbeatTimerCounter = 0;
11,570✔
1330

1331
  // init peer heartbeat timer
1332
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
185,120✔
1333
    if ((code = syncHbTimerInit(pSyncNode, &(pSyncNode->peerHeartbeatTimerArr[i]), (pSyncNode->replicasId)[i])) != 0) {
173,550!
1334
      errno = code;
×
1335
      goto _error;
×
1336
    }
1337
  }
1338

1339
  // tools
1340
  if ((code = syncRespMgrCreate(pSyncNode, SYNC_RESP_TTL_MS, &pSyncNode->pSyncRespMgr)) != 0) {
11,570!
1341
    sError("vgId:%d, failed to create SyncRespMgr", pSyncNode->vgId);
×
1342
    goto _error;
×
1343
  }
1344
  if (pSyncNode->pSyncRespMgr == NULL) {
11,570!
1345
    sError("vgId:%d, failed to create SyncRespMgr", pSyncNode->vgId);
×
1346
    goto _error;
×
1347
  }
1348

1349
  // restore state
1350
  pSyncNode->restoreFinish = false;
11,570✔
1351

1352
  // snapshot senders
1353
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
185,047✔
1354
    SSyncSnapshotSender* pSender = NULL;
173,473✔
1355
    code = snapshotSenderCreate(pSyncNode, i, &pSender);
173,473✔
1356
    if (pSender == NULL) return NULL;
173,476!
1357

1358
    pSyncNode->senders[i] = pSender;
173,476✔
1359
    sSDebug(pSender, "snapshot sender create while open sync node, data:%p", pSender);
173,476✔
1360
  }
1361

1362
  // snapshot receivers
1363
  code = snapshotReceiverCreate(pSyncNode, EMPTY_RAFT_ID, &pSyncNode->pNewNodeReceiver);
11,574✔
1364
  if (pSyncNode->pNewNodeReceiver == NULL) return NULL;
11,570!
1365
  sRDebug(pSyncNode->pNewNodeReceiver, "snapshot receiver create while open sync node, data:%p",
11,570✔
1366
          pSyncNode->pNewNodeReceiver);
1367

1368
  // is config changing
1369
  pSyncNode->changing = false;
11,570✔
1370

1371
  // replication mgr
1372
  if ((code = syncNodeLogReplInit(pSyncNode)) < 0) {
11,570!
1373
    terrno = code;
×
1374
    sError("vgId:%d, failed to init repl mgr since %s.", pSyncNode->vgId, terrstr());
×
1375
    goto _error;
×
1376
  }
1377

1378
  // peer state
1379
  if ((code = syncNodePeerStateInit(pSyncNode)) < 0) {
11,570!
1380
    terrno = code;
×
1381
    sError("vgId:%d, failed to init peer stat since %s.", pSyncNode->vgId, terrstr());
×
1382
    goto _error;
×
1383
  }
1384

1385
  //
1386
  // min match index
1387
  pSyncNode->minMatchIndex = SYNC_INDEX_INVALID;
11,569✔
1388

1389
  // start in syncNodeStart
1390
  // start raft
1391

1392
  int64_t timeNow = taosGetTimestampMs();
11,570✔
1393
  pSyncNode->startTime = timeNow;
11,570✔
1394
  pSyncNode->lastReplicateTime = timeNow;
11,570✔
1395

1396
  // snapshotting
1397
  atomic_store_64(&pSyncNode->snapshottingIndex, SYNC_INDEX_INVALID);
11,570✔
1398

1399
  // init log buffer
1400
  if ((code = syncLogBufferInit(pSyncNode->pLogBuf, pSyncNode)) < 0) {
11,570!
1401
    terrno = code;
×
1402
    sError("vgId:%d, failed to init sync log buffer since %s", pSyncNode->vgId, terrstr());
×
1403
    goto _error;
×
1404
  }
1405

1406
  pSyncNode->isStart = true;
11,570✔
1407
  pSyncNode->electNum = 0;
11,570✔
1408
  pSyncNode->becomeLeaderNum = 0;
11,570✔
1409
  pSyncNode->becomeAssignedLeaderNum = 0;
11,570✔
1410
  pSyncNode->configChangeNum = 0;
11,570✔
1411
  pSyncNode->hbSlowNum = 0;
11,570✔
1412
  pSyncNode->hbrSlowNum = 0;
11,570✔
1413
  pSyncNode->tmrRoutineNum = 0;
11,570✔
1414

1415
  sNInfo(pSyncNode, "sync node opened, node:%p electInterval:%d heartbeatInterval:%d heartbeatTimeout:%d", pSyncNode,
11,570✔
1416
         tsElectInterval, tsHeartbeatInterval, tsHeartbeatTimeout);
1417
  return pSyncNode;
11,570✔
1418

1419
_error:
×
1420
  if (pSyncInfo->pFsm) {
×
1421
    taosMemoryFree(pSyncInfo->pFsm);
×
1422
    pSyncInfo->pFsm = NULL;
×
1423
  }
1424
  syncNodeClose(pSyncNode);
×
1425
  pSyncNode = NULL;
×
1426
  return NULL;
×
1427
}
1428

1429
#ifdef BUILD_NO_CALL
1430
void syncNodeMaybeUpdateCommitBySnapshot(SSyncNode* pSyncNode) {
1431
  if (pSyncNode->pFsm != NULL && pSyncNode->pFsm->FpGetSnapshotInfo != NULL) {
1432
    SSnapshot snapshot = {0};
1433
    (void)pSyncNode->pFsm->FpGetSnapshotInfo(pSyncNode->pFsm, &snapshot);
1434
    if (snapshot.lastApplyIndex > pSyncNode->commitIndex) {
1435
      pSyncNode->commitIndex = snapshot.lastApplyIndex;
1436
    }
1437
  }
1438
}
1439
#endif
1440

1441
int32_t syncNodeRestore(SSyncNode* pSyncNode) {
11,569✔
1442
  int32_t code = 0;
11,569✔
1443
  if (pSyncNode->pLogStore == NULL) {
11,569!
1444
    sError("vgId:%d, log store not created", pSyncNode->vgId);
×
1445
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1446
  }
1447
  if (pSyncNode->pLogBuf == NULL) {
11,569!
1448
    sError("vgId:%d, ring log buffer not created", pSyncNode->vgId);
×
1449
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1450
  }
1451

1452
  (void)taosThreadMutexLock(&pSyncNode->pLogBuf->mutex);
11,569✔
1453
  SyncIndex lastVer = pSyncNode->pLogStore->syncLogLastIndex(pSyncNode->pLogStore);
11,569✔
1454
  SyncIndex commitIndex = pSyncNode->pLogStore->syncLogCommitIndex(pSyncNode->pLogStore);
11,569✔
1455
  SyncIndex endIndex = pSyncNode->pLogBuf->endIndex;
11,569✔
1456
  (void)taosThreadMutexUnlock(&pSyncNode->pLogBuf->mutex);
11,569✔
1457

1458
  if (lastVer != -1 && endIndex != lastVer + 1) {
11,569!
1459
    code = TSDB_CODE_WAL_LOG_INCOMPLETE;
×
1460
    sWarn("vgId:%d, failed to restore sync node since %s. expected lastLogIndex:%" PRId64 ", lastVer:%" PRId64 "",
×
1461
          pSyncNode->vgId, terrstr(), endIndex - 1, lastVer);
1462
    // TAOS_RETURN(code);
1463
  }
1464

1465
  // if (endIndex != lastVer + 1) return TSDB_CODE_SYN_INTERNAL_ERROR;
1466
  pSyncNode->commitIndex = TMAX(pSyncNode->commitIndex, commitIndex);
11,569✔
1467
  sInfo("vgId:%d, restore began, and keep syncing until commitIndex:%" PRId64, pSyncNode->vgId, pSyncNode->commitIndex);
11,569✔
1468

1469
  if (pSyncNode->fsmState != SYNC_FSM_STATE_INCOMPLETE &&
23,138!
1470
      (code = syncLogBufferCommit(pSyncNode->pLogBuf, pSyncNode, pSyncNode->commitIndex)) < 0) {
11,569✔
1471
    TAOS_RETURN(code);
×
1472
  }
1473

1474
  TAOS_RETURN(code);
11,569✔
1475
}
1476

1477
int32_t syncNodeStart(SSyncNode* pSyncNode) {
11,569✔
1478
  // start raft
1479
  sInfo("vgId:%d, begin to start sync node", pSyncNode->vgId);
11,569✔
1480
  if (pSyncNode->raftCfg.cfg.nodeInfo[pSyncNode->raftCfg.cfg.myIndex].nodeRole == TAOS_SYNC_ROLE_LEARNER) {
11,569✔
1481
    syncNodeBecomeLearner(pSyncNode, "first start");
243✔
1482
  } else {
1483
    if (pSyncNode->replicaNum == 1) {
11,326✔
1484
      raftStoreNextTerm(pSyncNode);
9,024✔
1485
      syncNodeBecomeLeader(pSyncNode, "one replica start");
9,024✔
1486

1487
      // Raft 3.6.2 Committing entries from previous terms
1488
      TAOS_CHECK_RETURN(syncNodeAppendNoop(pSyncNode));
9,024✔
1489
    } else {
1490
      SRaftId id = {0};
2,302✔
1491
      syncNodeBecomeFollower(pSyncNode, id, "first start");
2,302✔
1492
    }
1493
  }
1494

1495
  int32_t ret = 0;
11,568✔
1496
  ret = syncNodeStartPingTimer(pSyncNode);
11,568✔
1497
  if (ret != 0) {
11,568!
1498
    sError("vgId:%d, failed to start ping timer since %s", pSyncNode->vgId, tstrerror(ret));
×
1499
  }
1500
  sInfo("vgId:%d, sync node started", pSyncNode->vgId);
11,568✔
1501
  return ret;
11,568✔
1502
}
1503

1504
#ifdef BUILD_NO_CALL
1505
int32_t syncNodeStartStandBy(SSyncNode* pSyncNode) {
1506
  // state change
1507
  int32_t code = 0;
1508
  pSyncNode->state = TAOS_SYNC_STATE_FOLLOWER;
1509
  pSyncNode->roleTimeMs = taosGetTimestampMs();
1510
  // TODO check return value
1511
  TAOS_CHECK_RETURN(syncNodeStopHeartbeatTimer(pSyncNode));
1512

1513
  // reset elect timer, long enough
1514
  int32_t electMS = TIMER_MAX_MS;
1515
  code = syncNodeRestartElectTimer(pSyncNode, electMS);
1516
  if (code < 0) {
1517
    sError("vgId:%d, failed to restart elect timer since %s", pSyncNode->vgId, terrstr());
1518
    return -1;
1519
  }
1520

1521
  code = syncNodeStartPingTimer(pSyncNode);
1522
  if (code < 0) {
1523
    sError("vgId:%d, failed to start ping timer since %s", pSyncNode->vgId, terrstr());
1524
    return -1;
1525
  }
1526
  return code;
1527
}
1528
#endif
1529

1530
void syncNodePreClose(SSyncNode* pSyncNode) {
11,569✔
1531
  int32_t code = 0;
11,569✔
1532
  if (pSyncNode == NULL) {
11,569!
1533
    sError("failed to pre close sync node since sync node is null");
×
1534
    return;
×
1535
  }
1536
  if (pSyncNode->pFsm == NULL) {
11,569!
1537
    sError("failed to pre close sync node since fsm is null");
×
1538
    return;
×
1539
  }
1540
  if (pSyncNode->pFsm->FpApplyQueueItems == NULL) {
11,569!
1541
    sError("failed to pre close sync node since FpApplyQueueItems is null");
×
1542
    return;
×
1543
  }
1544

1545
  // stop elect timer
1546
  if ((code = syncNodeStopElectTimer(pSyncNode)) != 0) {
11,569!
1547
    sError("vgId:%d, failed to stop elect timer since %s", pSyncNode->vgId, tstrerror(code));
×
1548
    return;
×
1549
  }
1550

1551
  // stop heartbeat timer
1552
  if ((code = syncNodeStopHeartbeatTimer(pSyncNode)) != 0) {
11,569!
1553
    sError("vgId:%d, failed to stop heartbeat timer since %s", pSyncNode->vgId, tstrerror(code));
×
1554
    return;
×
1555
  }
1556

1557
  // stop ping timer
1558
  if ((code = syncNodeStopPingTimer(pSyncNode)) != 0) {
11,569!
1559
    sError("vgId:%d, failed to stop ping timer since %s", pSyncNode->vgId, tstrerror(code));
×
1560
    return;
×
1561
  }
1562

1563
  // clean rsp
1564
  syncRespCleanRsp(pSyncNode->pSyncRespMgr);
11,569✔
1565
}
1566

1567
void syncNodePostClose(SSyncNode* pSyncNode) {
10,052✔
1568
  if (pSyncNode->pNewNodeReceiver != NULL) {
10,052!
1569
    if (snapshotReceiverIsStart(pSyncNode->pNewNodeReceiver)) {
10,052!
1570
      snapshotReceiverStop(pSyncNode->pNewNodeReceiver);
×
1571
    }
1572

1573
    sDebug("vgId:%d, snapshot receiver destroy while preclose sync node, data:%p", pSyncNode->vgId,
10,051✔
1574
           pSyncNode->pNewNodeReceiver);
1575
    snapshotReceiverDestroy(pSyncNode->pNewNodeReceiver);
10,051✔
1576
    pSyncNode->pNewNodeReceiver = NULL;
10,052✔
1577
  }
1578
}
10,052✔
1579

1580
void syncHbTimerDataFree(SSyncHbTimerData* pData) { taosMemoryFree(pData); }
1,888!
1581

1582
void syncNodeClose(SSyncNode* pSyncNode) {
11,569✔
1583
  int32_t code = 0;
11,569✔
1584
  if (pSyncNode == NULL) return;
11,569!
1585
  sNInfo(pSyncNode, "sync close, node:%p", pSyncNode);
11,569✔
1586

1587
  syncRespCleanRsp(pSyncNode->pSyncRespMgr);
11,569✔
1588

1589
  if ((code = syncNodeStopPingTimer(pSyncNode)) != 0) {
11,568!
1590
    sError("vgId:%d, failed to stop ping timer since %s", pSyncNode->vgId, tstrerror(code));
×
1591
    return;
×
1592
  }
1593
  if ((code = syncNodeStopElectTimer(pSyncNode)) != 0) {
11,568!
1594
    sError("vgId:%d, failed to stop elect timer since %s", pSyncNode->vgId, tstrerror(code));
×
1595
    return;
×
1596
  }
1597
  if ((code = syncNodeStopHeartbeatTimer(pSyncNode)) != 0) {
11,569!
1598
    sError("vgId:%d, failed to stop heartbeat timer since %s", pSyncNode->vgId, tstrerror(code));
×
1599
    return;
×
1600
  }
1601
  syncNodeLogReplDestroy(pSyncNode);
11,568✔
1602

1603
  syncRespMgrDestroy(pSyncNode->pSyncRespMgr);
11,569✔
1604
  pSyncNode->pSyncRespMgr = NULL;
11,569✔
1605
  voteGrantedDestroy(pSyncNode->pVotesGranted);
11,569✔
1606
  pSyncNode->pVotesGranted = NULL;
11,569✔
1607
  votesRespondDestory(pSyncNode->pVotesRespond);
11,569✔
1608
  pSyncNode->pVotesRespond = NULL;
11,568✔
1609
  syncIndexMgrDestroy(pSyncNode->pNextIndex);
11,568✔
1610
  pSyncNode->pNextIndex = NULL;
11,568✔
1611
  syncIndexMgrDestroy(pSyncNode->pMatchIndex);
11,568✔
1612
  pSyncNode->pMatchIndex = NULL;
11,568✔
1613
  logStoreDestory(pSyncNode->pLogStore);
11,568✔
1614
  pSyncNode->pLogStore = NULL;
11,567✔
1615
  syncLogBufferDestroy(pSyncNode->pLogBuf);
11,567✔
1616
  pSyncNode->pLogBuf = NULL;
11,569✔
1617

1618
  (void)taosThreadMutexDestroy(&pSyncNode->arbTokenMutex);
11,569✔
1619

1620
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
185,069✔
1621
    if (pSyncNode->senders[i] != NULL) {
173,501!
1622
      sDebug("vgId:%d, snapshot sender destroy while close, data:%p", pSyncNode->vgId, pSyncNode->senders[i]);
173,501✔
1623

1624
      if (snapshotSenderIsStart(pSyncNode->senders[i])) {
173,501!
1625
        snapshotSenderStop(pSyncNode->senders[i], false);
×
1626
      }
1627

1628
      snapshotSenderDestroy(pSyncNode->senders[i]);
173,504✔
1629
      pSyncNode->senders[i] = NULL;
173,512✔
1630
    }
1631
  }
1632

1633
  if (pSyncNode->pNewNodeReceiver != NULL) {
11,568✔
1634
    if (snapshotReceiverIsStart(pSyncNode->pNewNodeReceiver)) {
1,517!
1635
      snapshotReceiverStop(pSyncNode->pNewNodeReceiver);
×
1636
    }
1637

1638
    sDebug("vgId:%d, snapshot receiver destroy while close, data:%p", pSyncNode->vgId, pSyncNode->pNewNodeReceiver);
1,517✔
1639
    snapshotReceiverDestroy(pSyncNode->pNewNodeReceiver);
1,517✔
1640
    pSyncNode->pNewNodeReceiver = NULL;
1,517✔
1641
  }
1642

1643
  if (pSyncNode->pFsm != NULL) {
11,568!
1644
    taosMemoryFree(pSyncNode->pFsm);
11,568!
1645
  }
1646

1647
  raftStoreClose(pSyncNode);
11,567✔
1648

1649
  taosMemoryFree(pSyncNode);
11,566!
1650
}
1651

1652
ESyncStrategy syncNodeStrategy(SSyncNode* pSyncNode) { return pSyncNode->raftCfg.snapshotStrategy; }
×
1653

1654
// timer control --------------
1655
int32_t syncNodeStartPingTimer(SSyncNode* pSyncNode) {
11,568✔
1656
  int32_t code = 0;
11,568✔
1657
  if (syncIsInit()) {
11,568!
1658
    bool stopped = taosTmrReset(pSyncNode->FpPingTimerCB, pSyncNode->pingTimerMS, (void*)pSyncNode->rid,
11,568✔
1659
                                syncEnv()->pTimerManager, &pSyncNode->pPingTimer);
11,568✔
1660
    if (stopped) {
11,568!
1661
      sError("vgId:%d, failed to reset ping timer, ms:%d", pSyncNode->vgId, pSyncNode->pingTimerMS);
×
1662
      return TSDB_CODE_SYN_INTERNAL_ERROR;
×
1663
    }
1664
    atomic_store_64(&pSyncNode->pingTimerLogicClock, pSyncNode->pingTimerLogicClockUser);
11,568✔
1665
  } else {
1666
    sError("vgId:%d, start ping timer error, sync env is stop", pSyncNode->vgId);
×
1667
  }
1668
  return code;
11,568✔
1669
}
1670

1671
int32_t syncNodeStopPingTimer(SSyncNode* pSyncNode) {
23,137✔
1672
  int32_t code = 0;
23,137✔
1673
  (void)atomic_add_fetch_64(&pSyncNode->pingTimerLogicClockUser, 1);
23,137✔
1674
  bool stop = taosTmrStop(pSyncNode->pPingTimer);
23,138✔
1675
  sDebug("vgId:%d, stop ping timer, stop:%d", pSyncNode->vgId, stop);
23,137✔
1676
  pSyncNode->pPingTimer = NULL;
23,137✔
1677
  return code;
23,137✔
1678
}
1679

1680
int32_t syncNodeStartElectTimer(SSyncNode* pSyncNode, int32_t ms) {
478,716✔
1681
  int32_t code = 0;
478,716✔
1682
  if (syncIsInit()) {
478,716!
1683
    pSyncNode->electTimerMS = ms;
478,716✔
1684

1685
    int64_t execTime = taosGetTimestampMs() + ms;
478,716✔
1686
    atomic_store_64(&(pSyncNode->electTimerParam.executeTime), execTime);
478,716✔
1687
    atomic_store_64(&(pSyncNode->electTimerParam.logicClock), pSyncNode->electTimerLogicClock);
478,716✔
1688
    pSyncNode->electTimerParam.pSyncNode = pSyncNode;
478,717✔
1689
    pSyncNode->electTimerParam.pData = NULL;
478,717✔
1690

1691
    bool stopped = taosTmrReset(pSyncNode->FpElectTimerCB, pSyncNode->electTimerMS, (void*)(pSyncNode->rid),
478,717✔
1692
                                syncEnv()->pTimerManager, &pSyncNode->pElectTimer);
478,717✔
1693
    if (stopped) sError("vgId:%d, failed to reset elect timer, ms:%d", pSyncNode->vgId, ms);
478,717!
1694
  } else {
1695
    sError("vgId:%d, start elect timer error, sync env is stop", pSyncNode->vgId);
×
1696
  }
1697
  return code;
478,717✔
1698
}
1699

1700
int32_t syncNodeStopElectTimer(SSyncNode* pSyncNode) {
511,653✔
1701
  int32_t code = 0;
511,653✔
1702
  (void)atomic_add_fetch_64(&pSyncNode->electTimerLogicClock, 1);
511,653✔
1703
  bool stop = taosTmrStop(pSyncNode->pElectTimer);
511,654✔
1704
  sDebug("vgId:%d, stop elect timer, stop:%d", pSyncNode->vgId, stop);
511,653✔
1705
  pSyncNode->pElectTimer = NULL;
511,653✔
1706

1707
  return code;
511,653✔
1708
}
1709

1710
int32_t syncNodeRestartElectTimer(SSyncNode* pSyncNode, int32_t ms) {
478,717✔
1711
  int32_t ret = 0;
478,717✔
1712
  TAOS_CHECK_RETURN(syncNodeStopElectTimer(pSyncNode));
478,717!
1713
  TAOS_CHECK_RETURN(syncNodeStartElectTimer(pSyncNode, ms));
478,716!
1714
  return ret;
478,717✔
1715
}
1716

1717
void syncNodeResetElectTimer(SSyncNode* pSyncNode) {
478,715✔
1718
  int32_t code = 0;
478,715✔
1719
  int32_t electMS;
1720

1721
  if (pSyncNode->raftCfg.isStandBy) {
478,715!
1722
    electMS = TIMER_MAX_MS;
×
1723
  } else {
1724
    electMS = syncUtilElectRandomMS(pSyncNode->electBaseLine, 2 * pSyncNode->electBaseLine);
478,715✔
1725
  }
1726

1727
  if ((code = syncNodeRestartElectTimer(pSyncNode, electMS)) != 0) {
478,717!
1728
    sWarn("vgId:%d, failed to restart elect timer since %s", pSyncNode->vgId, tstrerror(code));
×
1729
    return;
×
1730
  };
1731

1732
  sNTrace(pSyncNode, "reset elect timer, min:%d, max:%d, ms:%d", pSyncNode->electBaseLine, 2 * pSyncNode->electBaseLine,
478,717!
1733
          electMS);
1734
}
1735

1736
#ifdef BUILD_NO_CALL
1737
static int32_t syncNodeDoStartHeartbeatTimer(SSyncNode* pSyncNode) {
1738
  int32_t code = 0;
1739
  if (syncIsInit()) {
1740
    TAOS_CHECK_RETURN(taosTmrReset(pSyncNode->FpHeartbeatTimerCB, pSyncNode->heartbeatTimerMS, (void*)pSyncNode->rid,
1741
                                   syncEnv()->pTimerManager, &pSyncNode->pHeartbeatTimer));
1742
    atomic_store_64(&pSyncNode->heartbeatTimerLogicClock, pSyncNode->heartbeatTimerLogicClockUser);
1743
  } else {
1744
    sError("vgId:%d, start heartbeat timer error, sync env is stop", pSyncNode->vgId);
1745
  }
1746

1747
  sNTrace(pSyncNode, "start heartbeat timer, ms:%d", pSyncNode->heartbeatTimerMS);
1748
  return code;
1749
}
1750
#endif
1751

1752
int32_t syncNodeStartHeartbeatTimer(SSyncNode* pSyncNode) {
11,076✔
1753
  int32_t ret = 0;
11,076✔
1754

1755
#if 0
1756
  pSyncNode->heartbeatTimerMS = pSyncNode->hbBaseLine;
1757
  ret = syncNodeDoStartHeartbeatTimer(pSyncNode);
1758
#endif
1759

1760
  for (int32_t i = 0; i < pSyncNode->peersNum; ++i) {
12,969✔
1761
    SSyncTimer* pSyncTimer = syncNodeGetHbTimer(pSyncNode, &(pSyncNode->peersId[i]));
1,893✔
1762
    if (pSyncTimer != NULL) {
1,893!
1763
      TAOS_CHECK_RETURN(syncHbTimerStart(pSyncNode, pSyncTimer));
1,893!
1764
    }
1765
  }
1766

1767
  return ret;
11,076✔
1768
}
1769

1770
int32_t syncNodeStopHeartbeatTimer(SSyncNode* pSyncNode) {
28,361✔
1771
  int32_t code = 0;
28,361✔
1772

1773
#if 0
1774
  TAOS_CHECK_RETURN(atomic_add_fetch_64(&pSyncNode->heartbeatTimerLogicClockUser, 1));
1775
  bool stop = taosTmrStop(pSyncNode->pHeartbeatTimer);
1776
  sDebug("vgId:%d, stop heartbeat timer, stop:%d", pSyncNode->vgId, stop);
1777
  pSyncNode->pHeartbeatTimer = NULL;
1778
#endif
1779

1780
  for (int32_t i = 0; i < pSyncNode->peersNum; ++i) {
47,977✔
1781
    SSyncTimer* pSyncTimer = syncNodeGetHbTimer(pSyncNode, &(pSyncNode->peersId[i]));
19,616✔
1782
    if (pSyncTimer != NULL) {
19,616!
1783
      TAOS_CHECK_RETURN(syncHbTimerStop(pSyncNode, pSyncTimer));
19,616!
1784
    }
1785
  }
1786

1787
  return code;
28,361✔
1788
}
1789

1790
#ifdef BUILD_NO_CALL
1791
int32_t syncNodeRestartHeartbeatTimer(SSyncNode* pSyncNode) {
1792
  // TODO check return value
1793
  int32_t code = 0;
1794
  TAOS_CHECK_RETURN(syncNodeStopHeartbeatTimer(pSyncNode));
1795
  TAOS_CHECK_RETURN(syncNodeStartHeartbeatTimer(pSyncNode));
1796
  return 0;
1797
}
1798
#endif
1799

1800
int32_t syncNodeSendMsgById(const SRaftId* destRaftId, SSyncNode* pNode, SRpcMsg* pMsg) {
5,631,881✔
1801
  SEpSet* epSet = NULL;
5,631,881✔
1802
  for (int32_t i = 0; i < pNode->peersNum; ++i) {
5,961,048✔
1803
    if (destRaftId->addr == pNode->peersId[i].addr) {
5,961,013✔
1804
      epSet = &pNode->peersEpset[i];
5,631,846✔
1805
      break;
5,631,846✔
1806
    }
1807
  }
1808

1809
  int32_t code = -1;
5,631,881✔
1810
  if (pNode->syncSendMSg != NULL && epSet != NULL) {
5,631,881!
1811
    syncUtilMsgHtoN(pMsg->pCont);
5,631,849✔
1812
    pMsg->info.noResp = 1;
5,631,841✔
1813
    code = pNode->syncSendMSg(epSet, pMsg);
5,631,841✔
1814
  }
1815

1816
  if (code < 0) {
5,631,896✔
1817
    sError("vgId:%d, failed to send sync msg since %s. epset:%p dnode:%d addr:%" PRId64, pNode->vgId, tstrerror(code),
46!
1818
           epSet, DID(destRaftId), destRaftId->addr);
1819
    rpcFreeCont(pMsg->pCont);
46✔
1820
  }
1821

1822
  TAOS_RETURN(code);
5,631,896✔
1823
}
1824

1825
inline bool syncNodeInConfig(SSyncNode* pNode, const SSyncCfg* pCfg) {
1,869✔
1826
  bool b1 = false;
1,869✔
1827
  bool b2 = false;
1,869✔
1828

1829
  for (int32_t i = 0; i < pCfg->totalReplicaNum; ++i) {
2,228!
1830
    if (strcmp(pCfg->nodeInfo[i].nodeFqdn, pNode->myNodeInfo.nodeFqdn) == 0 &&
2,228!
1831
        pCfg->nodeInfo[i].nodePort == pNode->myNodeInfo.nodePort) {
2,228✔
1832
      b1 = true;
1,869✔
1833
      break;
1,869✔
1834
    }
1835
  }
1836

1837
  for (int32_t i = 0; i < pCfg->totalReplicaNum; ++i) {
2,228!
1838
    SRaftId raftId = {
2,228✔
1839
        .addr = SYNC_ADDR(&pCfg->nodeInfo[i]),
2,228✔
1840
        .vgId = pNode->vgId,
2,228✔
1841
    };
1842

1843
    if (syncUtilSameId(&raftId, &pNode->myRaftId)) {
2,228✔
1844
      b2 = true;
1,869✔
1845
      break;
1,869✔
1846
    }
1847
  }
1848

1849
  if (b1 != b2) {
1,869!
1850
    terrno = TSDB_CODE_SYN_INTERNAL_ERROR;
×
1851
    return false;
×
1852
  }
1853
  return b1;
1,869✔
1854
}
1855

1856
static bool syncIsConfigChanged(const SSyncCfg* pOldCfg, const SSyncCfg* pNewCfg) {
2,810✔
1857
  if (pOldCfg->totalReplicaNum != pNewCfg->totalReplicaNum) return true;
2,810✔
1858
  if (pOldCfg->myIndex != pNewCfg->myIndex) return true;
1,855✔
1859
  for (int32_t i = 0; i < pOldCfg->totalReplicaNum; ++i) {
4,459✔
1860
    const SNodeInfo* pOldInfo = &pOldCfg->nodeInfo[i];
3,262✔
1861
    const SNodeInfo* pNewInfo = &pNewCfg->nodeInfo[i];
3,262✔
1862
    if (strcmp(pOldInfo->nodeFqdn, pNewInfo->nodeFqdn) != 0) return true;
3,262!
1863
    if (pOldInfo->nodePort != pNewInfo->nodePort) return true;
3,262✔
1864
    if (pOldInfo->nodeRole != pNewInfo->nodeRole) return true;
3,260✔
1865
  }
1866

1867
  return false;
1,197✔
1868
}
1869

1870
int32_t syncNodeDoConfigChange(SSyncNode* pSyncNode, SSyncCfg* pNewConfig, SyncIndex lastConfigChangeIndex) {
1,421✔
1871
  int32_t  code = 0;
1,421✔
1872
  SSyncCfg oldConfig = pSyncNode->raftCfg.cfg;
1,421✔
1873
  if (!syncIsConfigChanged(&oldConfig, pNewConfig)) {
1,421✔
1874
    sInfo("vgId:1, sync not reconfig since not changed");
1,197✔
1875
    return 0;
1,197✔
1876
  }
1877

1878
  pSyncNode->raftCfg.cfg = *pNewConfig;
224✔
1879
  pSyncNode->raftCfg.lastConfigIndex = lastConfigChangeIndex;
224✔
1880

1881
  pSyncNode->configChangeNum++;
224✔
1882

1883
  bool IamInOld = syncNodeInConfig(pSyncNode, &oldConfig);
224✔
1884
  bool IamInNew = syncNodeInConfig(pSyncNode, pNewConfig);
224✔
1885

1886
  bool isDrop = false;
224✔
1887
  bool isAdd = false;
224✔
1888

1889
  if (IamInOld && !IamInNew) {
224!
1890
    isDrop = true;
×
1891
  } else {
1892
    isDrop = false;
224✔
1893
  }
1894

1895
  if (!IamInOld && IamInNew) {
224!
1896
    isAdd = true;
×
1897
  } else {
1898
    isAdd = false;
224✔
1899
  }
1900

1901
  // log begin config change
1902
  sNInfo(pSyncNode, "begin do config change, from %d to %d, from %" PRId64 " to %" PRId64 ", replicas:%d",
224!
1903
         pSyncNode->vgId, oldConfig.totalReplicaNum, pNewConfig->totalReplicaNum, oldConfig.lastIndex,
1904
         pNewConfig->lastIndex);
1905

1906
  if (IamInNew) {
224!
1907
    pSyncNode->raftCfg.isStandBy = 0;  // change isStandBy to normal
224✔
1908
  }
1909
  if (isDrop) {
224!
1910
    pSyncNode->raftCfg.isStandBy = 1;  // set standby
×
1911
  }
1912

1913
  // add last config index
1914
  SRaftCfg* pCfg = &pSyncNode->raftCfg;
224✔
1915
  if (pCfg->configIndexCount >= MAX_CONFIG_INDEX_COUNT) {
224!
1916
    sNError(pSyncNode, "failed to add cfg index:%d since out of range", pCfg->configIndexCount);
×
1917
    terrno = TSDB_CODE_OUT_OF_RANGE;
×
1918
    return -1;
×
1919
  }
1920

1921
  pCfg->configIndexArr[pCfg->configIndexCount] = lastConfigChangeIndex;
224✔
1922
  pCfg->configIndexCount++;
224✔
1923

1924
  if (IamInNew) {
224!
1925
    //-----------------------------------------
1926
    int32_t ret = 0;
224✔
1927

1928
    // save snapshot senders
1929
    SRaftId oldReplicasId[TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA];
1930
    memcpy(oldReplicasId, pSyncNode->replicasId, sizeof(oldReplicasId));
224✔
1931
    SSyncSnapshotSender* oldSenders[TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA];
1932
    for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
3,584✔
1933
      oldSenders[i] = pSyncNode->senders[i];
3,360✔
1934
      sSTrace(oldSenders[i], "snapshot sender save old");
3,360!
1935
    }
1936

1937
    // init internal
1938
    pSyncNode->myNodeInfo = pSyncNode->raftCfg.cfg.nodeInfo[pSyncNode->raftCfg.cfg.myIndex];
224✔
1939
    if (syncUtilNodeInfo2RaftId(&pSyncNode->myNodeInfo, pSyncNode->vgId, &pSyncNode->myRaftId) == false) return terrno;
224!
1940

1941
    // init peersNum, peers, peersId
1942
    pSyncNode->peersNum = pSyncNode->raftCfg.cfg.totalReplicaNum - 1;
224✔
1943
    int32_t j = 0;
224✔
1944
    for (int32_t i = 0; i < pSyncNode->raftCfg.cfg.totalReplicaNum; ++i) {
810✔
1945
      if (i != pSyncNode->raftCfg.cfg.myIndex) {
586✔
1946
        pSyncNode->peersNodeInfo[j] = pSyncNode->raftCfg.cfg.nodeInfo[i];
362✔
1947
        syncUtilNodeInfo2EpSet(&pSyncNode->peersNodeInfo[j], &pSyncNode->peersEpset[j]);
362✔
1948
        j++;
362✔
1949
      }
1950
    }
1951
    for (int32_t i = 0; i < pSyncNode->peersNum; ++i) {
586✔
1952
      if (syncUtilNodeInfo2RaftId(&pSyncNode->peersNodeInfo[i], pSyncNode->vgId, &pSyncNode->peersId[i]) == false)
362!
1953
        return terrno;
×
1954
    }
1955

1956
    // init replicaNum, replicasId
1957
    pSyncNode->replicaNum = pSyncNode->raftCfg.cfg.replicaNum;
224✔
1958
    pSyncNode->totalReplicaNum = pSyncNode->raftCfg.cfg.totalReplicaNum;
224✔
1959
    for (int32_t i = 0; i < pSyncNode->raftCfg.cfg.totalReplicaNum; ++i) {
810✔
1960
      if (syncUtilNodeInfo2RaftId(&pSyncNode->raftCfg.cfg.nodeInfo[i], pSyncNode->vgId, &pSyncNode->replicasId[i]) ==
586!
1961
          false)
1962
        return terrno;
×
1963
    }
1964

1965
    // update quorum first
1966
    pSyncNode->quorum = syncUtilQuorum(pSyncNode->raftCfg.cfg.replicaNum);
224✔
1967

1968
    syncIndexMgrUpdate(pSyncNode->pNextIndex, pSyncNode);
224✔
1969
    syncIndexMgrUpdate(pSyncNode->pMatchIndex, pSyncNode);
224✔
1970
    voteGrantedUpdate(pSyncNode->pVotesGranted, pSyncNode);
224✔
1971
    votesRespondUpdate(pSyncNode->pVotesRespond, pSyncNode);
224✔
1972

1973
    // reset snapshot senders
1974

1975
    // clear new
1976
    for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
3,584✔
1977
      pSyncNode->senders[i] = NULL;
3,360✔
1978
    }
1979

1980
    // reset new
1981
    for (int32_t i = 0; i < pSyncNode->totalReplicaNum; ++i) {
810✔
1982
      // reset sender
1983
      bool reset = false;
586✔
1984
      for (int32_t j = 0; j < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++j) {
2,472✔
1985
        if (syncUtilSameId(&(pSyncNode->replicasId)[i], &oldReplicasId[j]) && oldSenders[j] != NULL) {
2,369!
1986
          sNTrace(pSyncNode, "snapshot sender reset for:%" PRId64 ", newIndex:%d, dnode:%d, %p",
483!
1987
                  (pSyncNode->replicasId)[i].addr, i, DID(&pSyncNode->replicasId[i]), oldSenders[j]);
1988

1989
          pSyncNode->senders[i] = oldSenders[j];
483✔
1990
          oldSenders[j] = NULL;
483✔
1991
          reset = true;
483✔
1992

1993
          // reset replicaIndex
1994
          int32_t oldreplicaIndex = pSyncNode->senders[i]->replicaIndex;
483✔
1995
          pSyncNode->senders[i]->replicaIndex = i;
483✔
1996

1997
          sNTrace(pSyncNode, "snapshot sender udpate replicaIndex from %d to %d, dnode:%d, %p, reset:%d",
483!
1998
                  oldreplicaIndex, i, DID(&pSyncNode->replicasId[i]), pSyncNode->senders[i], reset);
1999

2000
          break;
483✔
2001
        }
2002
      }
2003
    }
2004

2005
    // create new
2006
    for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
3,584✔
2007
      if (pSyncNode->senders[i] == NULL) {
3,360✔
2008
        TAOS_CHECK_RETURN(snapshotSenderCreate(pSyncNode, i, &pSyncNode->senders[i]));
2,877!
2009
        if (pSyncNode->senders[i] == NULL) {
2,877!
2010
          // will be created later while send snapshot
2011
          sSError(pSyncNode->senders[i], "snapshot sender create failed while reconfig");
×
2012
        } else {
2013
          sSDebug(pSyncNode->senders[i], "snapshot sender create while reconfig, data:%p", pSyncNode->senders[i]);
2,877!
2014
        }
2015
      } else {
2016
        sSDebug(pSyncNode->senders[i], "snapshot sender already exist, data:%p", pSyncNode->senders[i]);
483!
2017
      }
2018
    }
2019

2020
    // free old
2021
    for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
3,584✔
2022
      if (oldSenders[i] != NULL) {
3,360✔
2023
        sSDebug(oldSenders[i], "snapshot sender destroy old, data:%p replica-index:%d", oldSenders[i], i);
2,877!
2024
        snapshotSenderDestroy(oldSenders[i]);
2,877✔
2025
        oldSenders[i] = NULL;
2,877✔
2026
      }
2027
    }
2028

2029
    // persist cfg
2030
    TAOS_CHECK_RETURN(syncWriteCfgFile(pSyncNode));
224!
2031
  } else {
2032
    // persist cfg
2033
    TAOS_CHECK_RETURN(syncWriteCfgFile(pSyncNode));
×
2034
    sNInfo(pSyncNode, "do not config change from %d to %d", oldConfig.totalReplicaNum, pNewConfig->totalReplicaNum);
×
2035
  }
2036

2037
_END:
×
2038
  // log end config change
2039
  sNInfo(pSyncNode, "end do config change, from %d to %d", oldConfig.totalReplicaNum, pNewConfig->totalReplicaNum);
224!
2040
  return 0;
224✔
2041
}
2042

2043
// raft state change --------------
2044
void syncNodeUpdateTermWithoutStepDown(SSyncNode* pSyncNode, SyncTerm term) {
36,992✔
2045
  if (term > raftStoreGetTerm(pSyncNode)) {
36,992!
2046
    raftStoreSetTerm(pSyncNode, term);
×
2047
  }
2048
}
36,992✔
2049

2050
void syncNodeStepDown(SSyncNode* pSyncNode, SyncTerm newTerm, SRaftId id) {
407,593✔
2051
  SyncTerm currentTerm = raftStoreGetTerm(pSyncNode);
407,593✔
2052
  if (currentTerm > newTerm) {
407,593!
2053
    sNTrace(pSyncNode, "step down, ignore, new-term:%" PRId64 ", current-term:%" PRId64, newTerm, currentTerm);
×
2054
    return;
×
2055
  }
2056

2057
  do {
2058
    sNTrace(pSyncNode, "step down, new-term:%" PRId64 ", current-term:%" PRId64, newTerm, currentTerm);
407,593!
2059
  } while (0);
2060

2061
  if (pSyncNode->state == TAOS_SYNC_STATE_ASSIGNED_LEADER) {
407,593!
2062
    (void)taosThreadMutexLock(&pSyncNode->arbTokenMutex);
×
2063
    syncUtilGenerateArbToken(pSyncNode->myNodeInfo.nodeId, pSyncNode->vgId, pSyncNode->arbToken);
×
2064
    sInfo("vgId:%d, step down as assigned leader, new arbToken:%s", pSyncNode->vgId, pSyncNode->arbToken);
×
2065
    (void)taosThreadMutexUnlock(&pSyncNode->arbTokenMutex);
×
2066
  }
2067

2068
  if (currentTerm < newTerm) {
407,592✔
2069
    raftStoreSetTerm(pSyncNode, newTerm);
1,636✔
2070
    char tmpBuf[64];
2071
    snprintf(tmpBuf, sizeof(tmpBuf), "step down, update term to %" PRId64, newTerm);
1,636✔
2072
    syncNodeBecomeFollower(pSyncNode, id, tmpBuf);
1,636✔
2073
    raftStoreClearVote(pSyncNode);
1,636✔
2074
  } else {
2075
    if (pSyncNode->state != TAOS_SYNC_STATE_FOLLOWER) {
405,956✔
2076
      syncNodeBecomeFollower(pSyncNode, id, "step down");
5✔
2077
    }
2078
  }
2079
}
2080

2081
void syncNodeLeaderChangeRsp(SSyncNode* pSyncNode) { syncRespCleanRsp(pSyncNode->pSyncRespMgr); }
3,947✔
2082

2083
void syncNodeBecomeFollower(SSyncNode* pSyncNode, SRaftId leaderId, const char* debugStr) {
3,947✔
2084
  int32_t code = 0;  // maybe clear leader cache
3,947✔
2085
  if (pSyncNode->state == TAOS_SYNC_STATE_LEADER) {
3,947✔
2086
    pSyncNode->leaderCache = EMPTY_RAFT_ID;
8✔
2087
  }
2088

2089
  pSyncNode->hbSlowNum = 0;
3,947✔
2090

2091
  pSyncNode->leaderCache = leaderId;  // state change
3,947✔
2092
  pSyncNode->state = TAOS_SYNC_STATE_FOLLOWER;
3,947✔
2093
  pSyncNode->roleTimeMs = taosGetTimestampMs();
3,947✔
2094
  if ((code = syncNodeStopHeartbeatTimer(pSyncNode)) != 0) {
3,947!
2095
    sError("vgId:%d, failed to stop heartbeat timer since %s", pSyncNode->vgId, tstrerror(code));
×
2096
    return;
×
2097
  }
2098

2099
  // trace log
2100
  sNTrace(pSyncNode, "become follower %s", debugStr);
3,947!
2101

2102
  // send rsp to client
2103
  syncNodeLeaderChangeRsp(pSyncNode);
3,947✔
2104

2105
  // call back
2106
  if (pSyncNode->pFsm != NULL && pSyncNode->pFsm->FpBecomeFollowerCb != NULL) {
3,947!
2107
    pSyncNode->pFsm->FpBecomeFollowerCb(pSyncNode->pFsm);
3,947✔
2108
  }
2109

2110
  // min match index
2111
  pSyncNode->minMatchIndex = SYNC_INDEX_INVALID;
3,947✔
2112

2113
  // reset log buffer
2114
  if ((code = syncLogBufferReset(pSyncNode->pLogBuf, pSyncNode)) != 0) {
3,947!
2115
    sError("vgId:%d, failed to reset log buffer since %s", pSyncNode->vgId, tstrerror(code));
×
2116
    return;
×
2117
  }
2118

2119
  // reset elect timer
2120
  syncNodeResetElectTimer(pSyncNode);
3,947✔
2121

2122
  sInfo("vgId:%d, become follower. %s", pSyncNode->vgId, debugStr);
3,947!
2123
}
2124

2125
void syncNodeBecomeLearner(SSyncNode* pSyncNode, const char* debugStr) {
243✔
2126
  pSyncNode->hbSlowNum = 0;
243✔
2127

2128
  // state change
2129
  pSyncNode->state = TAOS_SYNC_STATE_LEARNER;
243✔
2130
  pSyncNode->roleTimeMs = taosGetTimestampMs();
243✔
2131

2132
  // trace log
2133
  sNTrace(pSyncNode, "become learner %s", debugStr);
243!
2134

2135
  // call back
2136
  if (pSyncNode->pFsm != NULL && pSyncNode->pFsm->FpBecomeLearnerCb != NULL) {
243!
2137
    pSyncNode->pFsm->FpBecomeLearnerCb(pSyncNode->pFsm);
243✔
2138
  }
2139

2140
  // min match index
2141
  pSyncNode->minMatchIndex = SYNC_INDEX_INVALID;
243✔
2142

2143
  // reset log buffer
2144
  int32_t code = 0;
243✔
2145
  if ((code = syncLogBufferReset(pSyncNode->pLogBuf, pSyncNode)) != 0) {
243!
2146
    sError("vgId:%d, failed to reset log buffer since %s", pSyncNode->vgId, tstrerror(code));
×
2147
    return;
×
2148
  };
2149
}
2150

2151
// TLA+ Spec
2152
// \* Candidate i transitions to leader.
2153
// BecomeLeader(i) ==
2154
//     /\ state[i] = Candidate
2155
//     /\ votesGranted[i] \in Quorum
2156
//     /\ state'      = [state EXCEPT ![i] = Leader]
2157
//     /\ nextIndex'  = [nextIndex EXCEPT ![i] =
2158
//                          [j \in Server |-> Len(log[i]) + 1]]
2159
//     /\ matchIndex' = [matchIndex EXCEPT ![i] =
2160
//                          [j \in Server |-> 0]]
2161
//     /\ elections'  = elections \cup
2162
//                          {[eterm     |-> currentTerm[i],
2163
//                            eleader   |-> i,
2164
//                            elog      |-> log[i],
2165
//                            evotes    |-> votesGranted[i],
2166
//                            evoterLog |-> voterLog[i]]}
2167
//     /\ UNCHANGED <<messages, currentTerm, votedFor, candidateVars, logVars>>
2168
//
2169
void syncNodeBecomeLeader(SSyncNode* pSyncNode, const char* debugStr) {
9,799✔
2170
  int32_t code = 0;
9,799✔
2171
  pSyncNode->becomeLeaderNum++;
9,799✔
2172
  pSyncNode->hbrSlowNum = 0;
9,799✔
2173

2174
  // reset restoreFinish
2175
  pSyncNode->restoreFinish = false;
9,799✔
2176

2177
  // state change
2178
  pSyncNode->state = TAOS_SYNC_STATE_LEADER;
9,799✔
2179
  pSyncNode->roleTimeMs = taosGetTimestampMs();
9,799✔
2180

2181
  // set leader cache
2182
  pSyncNode->leaderCache = pSyncNode->myRaftId;
9,799✔
2183

2184
  for (int32_t i = 0; i < pSyncNode->pNextIndex->replicaNum; ++i) {
21,111✔
2185
    SyncIndex lastIndex;
2186
    SyncTerm  lastTerm;
2187
    int32_t   code = syncNodeGetLastIndexTerm(pSyncNode, &lastIndex, &lastTerm);
11,312✔
2188
    if (code != 0) {
11,312!
2189
      sError("vgId:%d, failed to become leader since %s", pSyncNode->vgId, tstrerror(code));
×
2190
      return;
×
2191
    }
2192
    pSyncNode->pNextIndex->index[i] = lastIndex + 1;
11,312✔
2193
  }
2194

2195
  for (int32_t i = 0; i < pSyncNode->pMatchIndex->replicaNum; ++i) {
21,111✔
2196
    // maybe overwrite myself, no harm
2197
    // just do it!
2198
    pSyncNode->pMatchIndex->index[i] = SYNC_INDEX_INVALID;
11,312✔
2199
  }
2200

2201
  // init peer mgr
2202
  if ((code = syncNodePeerStateInit(pSyncNode)) != 0) {
9,799!
2203
    sError("vgId:%d, failed to init peer state since %s", pSyncNode->vgId, tstrerror(code));
×
2204
    return;
×
2205
  }
2206

2207
#if 0
2208
  // update sender private term
2209
  SSyncSnapshotSender* pMySender = syncNodeGetSnapshotSender(pSyncNode, &(pSyncNode->myRaftId));
2210
  if (pMySender != NULL) {
2211
    for (int32_t i = 0; i < pSyncNode->pMatchIndex->replicaNum; ++i) {
2212
      if (pSyncNode->senders[i]->privateTerm > pMySender->privateTerm) {
2213
        pMySender->privateTerm = pSyncNode->senders[i]->privateTerm;
2214
      }
2215
    }
2216
    (pMySender->privateTerm) += 100;
2217
  }
2218
#endif
2219

2220
  // close receiver
2221
  if (snapshotReceiverIsStart(pSyncNode->pNewNodeReceiver)) {
9,799!
2222
    snapshotReceiverStop(pSyncNode->pNewNodeReceiver);
×
2223
  }
2224

2225
  // stop elect timer
2226
  if ((code = syncNodeStopElectTimer(pSyncNode)) != 0) {
9,799!
2227
    sError("vgId:%d, failed to stop elect timer since %s", pSyncNode->vgId, tstrerror(code));
×
2228
    return;
×
2229
  }
2230

2231
  // start heartbeat timer
2232
  if ((code = syncNodeStartHeartbeatTimer(pSyncNode)) != 0) {
9,799!
2233
    sError("vgId:%d, failed to start heartbeat timer since %s", pSyncNode->vgId, tstrerror(code));
×
2234
    return;
×
2235
  }
2236

2237
  // send heartbeat right now
2238
  if ((code = syncNodeHeartbeatPeers(pSyncNode)) != 0) {
9,799!
2239
    sError("vgId:%d, failed to send heartbeat to peers since %s", pSyncNode->vgId, tstrerror(code));
×
2240
    return;
×
2241
  }
2242

2243
  // call back
2244
  if (pSyncNode->pFsm != NULL && pSyncNode->pFsm->FpBecomeLeaderCb != NULL) {
9,799!
2245
    pSyncNode->pFsm->FpBecomeLeaderCb(pSyncNode->pFsm);
9,799✔
2246
  }
2247

2248
  // min match index
2249
  pSyncNode->minMatchIndex = SYNC_INDEX_INVALID;
9,799✔
2250

2251
  // reset log buffer
2252
  if ((code = syncLogBufferReset(pSyncNode->pLogBuf, pSyncNode)) != 0) {
9,799!
2253
    sError("vgId:%d, failed to reset log buffer since %s", pSyncNode->vgId, tstrerror(code));
×
2254
    return;
×
2255
  }
2256

2257
  // trace log
2258
  sNInfo(pSyncNode, "become leader %s", debugStr);
9,799✔
2259
}
2260

2261
void syncNodeBecomeAssignedLeader(SSyncNode* pSyncNode) {
×
2262
  int32_t code = 0;
×
2263
  pSyncNode->becomeAssignedLeaderNum++;
×
2264
  pSyncNode->hbrSlowNum = 0;
×
2265

2266
  // reset restoreFinish
2267
  // pSyncNode->restoreFinish = false;
2268

2269
  // state change
2270
  pSyncNode->state = TAOS_SYNC_STATE_ASSIGNED_LEADER;
×
2271
  pSyncNode->roleTimeMs = taosGetTimestampMs();
×
2272

2273
  // set leader cache
2274
  pSyncNode->leaderCache = pSyncNode->myRaftId;
×
2275

2276
  for (int32_t i = 0; i < pSyncNode->pNextIndex->replicaNum; ++i) {
×
2277
    SyncIndex lastIndex;
2278
    SyncTerm  lastTerm;
2279
    int32_t   code = syncNodeGetLastIndexTerm(pSyncNode, &lastIndex, &lastTerm);
×
2280
    if (code != 0) {
×
2281
      sError("vgId:%d, failed to become assigned leader since %s", pSyncNode->vgId, tstrerror(code));
×
2282
      return;
×
2283
    }
2284
    pSyncNode->pNextIndex->index[i] = lastIndex + 1;
×
2285
  }
2286

2287
  for (int32_t i = 0; i < pSyncNode->pMatchIndex->replicaNum; ++i) {
×
2288
    // maybe overwrite myself, no harm
2289
    // just do it!
2290
    pSyncNode->pMatchIndex->index[i] = SYNC_INDEX_INVALID;
×
2291
  }
2292

2293
  // init peer mgr
2294
  if ((code = syncNodePeerStateInit(pSyncNode)) != 0) {
×
2295
    sError("vgId:%d, failed to init peer state since %s", pSyncNode->vgId, tstrerror(code));
×
2296
    return;
×
2297
  }
2298

2299
  // close receiver
2300
  if (snapshotReceiverIsStart(pSyncNode->pNewNodeReceiver)) {
×
2301
    snapshotReceiverStop(pSyncNode->pNewNodeReceiver);
×
2302
  }
2303

2304
  // stop elect timer
2305
  if ((code = syncNodeStopElectTimer(pSyncNode)) != 0) {
×
2306
    sError("vgId:%d, failed to stop elect timer since %s", pSyncNode->vgId, tstrerror(code));
×
2307
    return;
×
2308
  }
2309

2310
  // start heartbeat timer
2311
  if ((code = syncNodeStartHeartbeatTimer(pSyncNode)) != 0) {
×
2312
    sError("vgId:%d, failed to start heartbeat timer since %s", pSyncNode->vgId, tstrerror(code));
×
2313
    return;
×
2314
  }
2315

2316
  // send heartbeat right now
2317
  if ((code = syncNodeHeartbeatPeers(pSyncNode)) != 0) {
×
2318
    sError("vgId:%d, failed to send heartbeat to peers since %s", pSyncNode->vgId, tstrerror(code));
×
2319
    return;
×
2320
  }
2321

2322
  // call back
2323
  if (pSyncNode->pFsm != NULL && pSyncNode->pFsm->FpBecomeAssignedLeaderCb != NULL) {
×
2324
    pSyncNode->pFsm->FpBecomeAssignedLeaderCb(pSyncNode->pFsm);
×
2325
  }
2326

2327
  // min match index
2328
  pSyncNode->minMatchIndex = SYNC_INDEX_INVALID;
×
2329

2330
  // reset log buffer
2331
  if ((code = syncLogBufferReset(pSyncNode->pLogBuf, pSyncNode)) != 0) {
×
2332
    sError("vgId:%d, failed to reset log buffer since %s", pSyncNode->vgId, tstrerror(code));
×
2333
    return;
×
2334
  }
2335

2336
  // trace log
2337
  sNInfo(pSyncNode, "become assigned leader");
×
2338
}
2339

2340
void syncNodeCandidate2Leader(SSyncNode* pSyncNode) {
775✔
2341
  if (pSyncNode->state != TAOS_SYNC_STATE_CANDIDATE) {
775!
2342
    sError("vgId:%d, failed leader from candidate since node state is wrong:%d", pSyncNode->vgId, pSyncNode->state);
×
2343
    return;
×
2344
  }
2345
  bool granted = voteGrantedMajority(pSyncNode->pVotesGranted);
775✔
2346
  if (!granted) {
775!
2347
    sError("vgId:%d, not granted by majority.", pSyncNode->vgId);
×
2348
    return;
×
2349
  }
2350
  syncNodeBecomeLeader(pSyncNode, "candidate to leader");
775✔
2351

2352
  sNTrace(pSyncNode, "state change syncNodeCandidate2Leader");
775!
2353

2354
  int32_t ret = syncNodeAppendNoop(pSyncNode);
775✔
2355
  if (ret < 0) {
775!
2356
    sError("vgId:%d, failed to append noop entry since %s", pSyncNode->vgId, terrstr());
×
2357
  }
2358

2359
  SyncIndex lastIndex = pSyncNode->pLogStore->syncLogLastIndex(pSyncNode->pLogStore);
775✔
2360

2361
  sInfo("vgId:%d, become leader. term:%" PRId64 ", commit index:%" PRId64 ", last index:%" PRId64 "", pSyncNode->vgId,
775!
2362
        raftStoreGetTerm(pSyncNode), pSyncNode->commitIndex, lastIndex);
2363
}
2364

2365
bool syncNodeIsMnode(SSyncNode* pSyncNode) { return (pSyncNode->vgId == 1); }
105,088✔
2366

2367
int32_t syncNodePeerStateInit(SSyncNode* pSyncNode) {
21,366✔
2368
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
341,880✔
2369
    pSyncNode->peerStates[i].lastSendIndex = SYNC_INDEX_INVALID;
320,514✔
2370
    pSyncNode->peerStates[i].lastSendTime = 0;
320,514✔
2371
  }
2372

2373
  return 0;
21,366✔
2374
}
2375

2376
void syncNodeFollower2Candidate(SSyncNode* pSyncNode) {
916✔
2377
  if (pSyncNode->state != TAOS_SYNC_STATE_FOLLOWER) {
916!
2378
    sError("vgId:%d, failed candidate from follower since node state is wrong:%d", pSyncNode->vgId, pSyncNode->state);
×
2379
    return;
×
2380
  }
2381
  pSyncNode->state = TAOS_SYNC_STATE_CANDIDATE;
916✔
2382
  pSyncNode->roleTimeMs = taosGetTimestampMs();
916✔
2383
  SyncIndex lastIndex = pSyncNode->pLogStore->syncLogLastIndex(pSyncNode->pLogStore);
916✔
2384
  sInfo("vgId:%d, become candidate from follower. term:%" PRId64 ", commit index:%" PRId64 ", last index:%" PRId64,
916!
2385
        pSyncNode->vgId, raftStoreGetTerm(pSyncNode), pSyncNode->commitIndex, lastIndex);
2386

2387
  sNTrace(pSyncNode, "follower to candidate");
916!
2388
}
2389

2390
int32_t syncNodeAssignedLeader2Leader(SSyncNode* pSyncNode) {
×
2391
  if (pSyncNode->state != TAOS_SYNC_STATE_ASSIGNED_LEADER) return TSDB_CODE_SYN_INTERNAL_ERROR;
×
2392
  syncNodeBecomeLeader(pSyncNode, "assigned leader to leader");
×
2393

2394
  sNTrace(pSyncNode, "assigned leader to leader");
×
2395

2396
  int32_t ret = syncNodeAppendNoop(pSyncNode);
×
2397
  if (ret < 0) {
×
2398
    sError("vgId:%d, failed to append noop entry since %s", pSyncNode->vgId, tstrerror(ret));
×
2399
  }
2400

2401
  SyncIndex lastIndex = pSyncNode->pLogStore->syncLogLastIndex(pSyncNode->pLogStore);
×
2402
  sInfo("vgId:%d, become leader from assigned leader. term:%" PRId64 ", commit index:%" PRId64
×
2403
        "assigned commit index:%" PRId64 ", last index:%" PRId64,
2404
        pSyncNode->vgId, raftStoreGetTerm(pSyncNode), pSyncNode->commitIndex, pSyncNode->assignedCommitIndex,
2405
        lastIndex);
2406
  return 0;
×
2407
}
2408

2409
// just called by syncNodeVoteForSelf
2410
void syncNodeVoteForTerm(SSyncNode* pSyncNode, SyncTerm term, SRaftId* pRaftId) {
964✔
2411
  SyncTerm storeTerm = raftStoreGetTerm(pSyncNode);
964✔
2412
  if (term != storeTerm) {
964!
2413
    sError("vgId:%d, failed to vote for term, term:%" PRId64 ", storeTerm:%" PRId64, pSyncNode->vgId, term, storeTerm);
×
2414
    return;
×
2415
  }
2416
  bool voted = raftStoreHasVoted(pSyncNode);
964✔
2417
  if (voted) {
964!
2418
    sError("vgId:%d, failed to vote for term since not voted", pSyncNode->vgId);
×
2419
    return;
×
2420
  }
2421

2422
  raftStoreVote(pSyncNode, pRaftId);
964✔
2423
}
2424

2425
// simulate get vote from outside
2426
void syncNodeVoteForSelf(SSyncNode* pSyncNode, SyncTerm currentTerm) {
964✔
2427
  syncNodeVoteForTerm(pSyncNode, currentTerm, &pSyncNode->myRaftId);
964✔
2428

2429
  SRpcMsg rpcMsg = {0};
964✔
2430
  int32_t ret = syncBuildRequestVoteReply(&rpcMsg, pSyncNode->vgId);
964✔
2431
  if (ret != 0) return;
964!
2432

2433
  SyncRequestVoteReply* pMsg = rpcMsg.pCont;
964✔
2434
  pMsg->srcId = pSyncNode->myRaftId;
964✔
2435
  pMsg->destId = pSyncNode->myRaftId;
964✔
2436
  pMsg->term = currentTerm;
964✔
2437
  pMsg->voteGranted = true;
964✔
2438

2439
  voteGrantedVote(pSyncNode->pVotesGranted, pMsg);
964✔
2440
  votesRespondAdd(pSyncNode->pVotesRespond, pMsg);
964✔
2441
  rpcFreeCont(rpcMsg.pCont);
964✔
2442
}
2443

2444
// return if has a snapshot
2445
bool syncNodeHasSnapshot(SSyncNode* pSyncNode) {
14,801✔
2446
  bool      ret = false;
14,801✔
2447
  SSnapshot snapshot = {.data = NULL, .lastApplyIndex = -1, .lastApplyTerm = 0, .lastConfigIndex = -1};
14,801✔
2448
  if (pSyncNode->pFsm->FpGetSnapshotInfo != NULL) {
14,801!
2449
    // TODO check return value
2450
    (void)pSyncNode->pFsm->FpGetSnapshotInfo(pSyncNode->pFsm, &snapshot);
14,801✔
2451
    if (snapshot.lastApplyIndex >= SYNC_INDEX_BEGIN) {
14,801✔
2452
      ret = true;
1,910✔
2453
    }
2454
  }
2455
  return ret;
14,801✔
2456
}
2457

2458
// return max(logLastIndex, snapshotLastIndex)
2459
// if no snapshot and log, return -1
2460
SyncIndex syncNodeGetLastIndex(const SSyncNode* pSyncNode) {
14,801✔
2461
  SSnapshot snapshot = {.data = NULL, .lastApplyIndex = -1, .lastApplyTerm = 0, .lastConfigIndex = -1};
14,801✔
2462
  if (pSyncNode->pFsm->FpGetSnapshotInfo != NULL) {
14,801!
2463
    // TODO check return value
2464
    (void)pSyncNode->pFsm->FpGetSnapshotInfo(pSyncNode->pFsm, &snapshot);
14,801✔
2465
  }
2466
  SyncIndex logLastIndex = pSyncNode->pLogStore->syncLogLastIndex(pSyncNode->pLogStore);
14,801✔
2467

2468
  SyncIndex lastIndex = logLastIndex > snapshot.lastApplyIndex ? logLastIndex : snapshot.lastApplyIndex;
14,801✔
2469
  return lastIndex;
14,801✔
2470
}
2471

2472
// return the last term of snapshot and log
2473
// if error, return SYNC_TERM_INVALID (by syncLogLastTerm)
2474
SyncTerm syncNodeGetLastTerm(SSyncNode* pSyncNode) {
14,801✔
2475
  SyncTerm lastTerm = 0;
14,801✔
2476
  if (syncNodeHasSnapshot(pSyncNode)) {
14,801✔
2477
    // has snapshot
2478
    SSnapshot snapshot = {.data = NULL, .lastApplyIndex = -1, .lastApplyTerm = 0, .lastConfigIndex = -1};
1,910✔
2479
    if (pSyncNode->pFsm->FpGetSnapshotInfo != NULL) {
1,910!
2480
      // TODO check return value
2481
      (void)pSyncNode->pFsm->FpGetSnapshotInfo(pSyncNode->pFsm, &snapshot);
1,910✔
2482
    }
2483

2484
    SyncIndex logLastIndex = pSyncNode->pLogStore->syncLogLastIndex(pSyncNode->pLogStore);
1,910✔
2485
    if (logLastIndex > snapshot.lastApplyIndex) {
1,910✔
2486
      lastTerm = pSyncNode->pLogStore->syncLogLastTerm(pSyncNode->pLogStore);
1,135✔
2487
    } else {
2488
      lastTerm = snapshot.lastApplyTerm;
775✔
2489
    }
2490

2491
  } else {
2492
    // no snapshot
2493
    lastTerm = pSyncNode->pLogStore->syncLogLastTerm(pSyncNode->pLogStore);
12,891✔
2494
  }
2495

2496
  return lastTerm;
14,801✔
2497
}
2498

2499
// get last index and term along with snapshot
2500
int32_t syncNodeGetLastIndexTerm(SSyncNode* pSyncNode, SyncIndex* pLastIndex, SyncTerm* pLastTerm) {
13,134✔
2501
  *pLastIndex = syncNodeGetLastIndex(pSyncNode);
13,134✔
2502
  *pLastTerm = syncNodeGetLastTerm(pSyncNode);
13,134✔
2503
  return 0;
13,134✔
2504
}
2505

2506
#ifdef BUILD_NO_CALL
2507
// return append-entries first try index
2508
SyncIndex syncNodeSyncStartIndex(SSyncNode* pSyncNode) {
2509
  SyncIndex syncStartIndex = syncNodeGetLastIndex(pSyncNode) + 1;
2510
  return syncStartIndex;
2511
}
2512

2513
// if index > 0, return index - 1
2514
// else, return -1
2515
SyncIndex syncNodeGetPreIndex(SSyncNode* pSyncNode, SyncIndex index) {
2516
  SyncIndex preIndex = index - 1;
2517
  if (preIndex < SYNC_INDEX_INVALID) {
2518
    preIndex = SYNC_INDEX_INVALID;
2519
  }
2520

2521
  return preIndex;
2522
}
2523

2524
// if index < 0, return SYNC_TERM_INVALID
2525
// if index == 0, return 0
2526
// if index > 0, return preTerm
2527
// if error, return SYNC_TERM_INVALID
2528
SyncTerm syncNodeGetPreTerm(SSyncNode* pSyncNode, SyncIndex index) {
2529
  if (index < SYNC_INDEX_BEGIN) {
2530
    return SYNC_TERM_INVALID;
2531
  }
2532

2533
  if (index == SYNC_INDEX_BEGIN) {
2534
    return 0;
2535
  }
2536

2537
  SyncTerm  preTerm = 0;
2538
  SyncIndex preIndex = index - 1;
2539

2540
  SSyncRaftEntry* pPreEntry = NULL;
2541
  SLRUCache*      pCache = pSyncNode->pLogStore->pCache;
2542
  LRUHandle*      h = taosLRUCacheLookup(pCache, &preIndex, sizeof(preIndex));
2543
  int32_t         code = 0;
2544
  if (h) {
2545
    pPreEntry = (SSyncRaftEntry*)taosLRUCacheValue(pCache, h);
2546
    code = 0;
2547

2548
    pSyncNode->pLogStore->cacheHit++;
2549
    sNTrace(pSyncNode, "hit cache index:%" PRId64 ", bytes:%u, %p", preIndex, pPreEntry->bytes, pPreEntry);
2550

2551
  } else {
2552
    pSyncNode->pLogStore->cacheMiss++;
2553
    sNTrace(pSyncNode, "miss cache index:%" PRId64, preIndex);
2554

2555
    code = pSyncNode->pLogStore->syncLogGetEntry(pSyncNode->pLogStore, preIndex, &pPreEntry);
2556
  }
2557

2558
  SSnapshot snapshot = {.data = NULL,
2559
                        .lastApplyIndex = SYNC_INDEX_INVALID,
2560
                        .lastApplyTerm = SYNC_TERM_INVALID,
2561
                        .lastConfigIndex = SYNC_INDEX_INVALID};
2562

2563
  if (code == 0) {
2564
    if (pPreEntry == NULL) return -1;
2565
    preTerm = pPreEntry->term;
2566

2567
    if (h) {
2568
      taosLRUCacheRelease(pCache, h, false);
2569
    } else {
2570
      syncEntryDestroy(pPreEntry);
2571
    }
2572

2573
    return preTerm;
2574
  } else {
2575
    if (pSyncNode->pFsm->FpGetSnapshotInfo != NULL) {
2576
      // TODO check return value
2577
      (void)pSyncNode->pFsm->FpGetSnapshotInfo(pSyncNode->pFsm, &snapshot);
2578
      if (snapshot.lastApplyIndex == preIndex) {
2579
        return snapshot.lastApplyTerm;
2580
      }
2581
    }
2582
  }
2583

2584
  sNError(pSyncNode, "sync node get pre term error, index:%" PRId64 ", snap-index:%" PRId64 ", snap-term:%" PRId64,
2585
          index, snapshot.lastApplyIndex, snapshot.lastApplyTerm);
2586
  return SYNC_TERM_INVALID;
2587
}
2588

2589
// get pre index and term of "index"
2590
int32_t syncNodeGetPreIndexTerm(SSyncNode* pSyncNode, SyncIndex index, SyncIndex* pPreIndex, SyncTerm* pPreTerm) {
2591
  *pPreIndex = syncNodeGetPreIndex(pSyncNode, index);
2592
  *pPreTerm = syncNodeGetPreTerm(pSyncNode, index);
2593
  return 0;
2594
}
2595
#endif
2596

2597
static void syncNodeEqPingTimer(void* param, void* tmrId) {
44,309✔
2598
  if (!syncIsInit()) return;
44,309!
2599

2600
  int64_t    rid = (int64_t)param;
44,309✔
2601
  SSyncNode* pNode = syncNodeAcquire(rid);
44,309✔
2602

2603
  if (pNode == NULL) return;
44,309!
2604

2605
  if (atomic_load_64(&pNode->pingTimerLogicClockUser) <= atomic_load_64(&pNode->pingTimerLogicClock)) {
44,309!
2606
    SRpcMsg rpcMsg = {0};
44,309✔
2607
    int32_t code = syncBuildTimeout(&rpcMsg, SYNC_TIMEOUT_PING, atomic_load_64(&pNode->pingTimerLogicClock),
44,309✔
2608
                                    pNode->pingTimerMS, pNode);
2609
    if (code != 0) {
44,309!
2610
      sError("failed to build ping msg");
×
2611
      rpcFreeCont(rpcMsg.pCont);
×
2612
      goto _out;
×
2613
    }
2614

2615
    // sTrace("enqueue ping msg");
2616
    code = pNode->syncEqMsg(pNode->msgcb, &rpcMsg);
44,309✔
2617
    if (code != 0) {
44,309✔
2618
      sError("failed to sync enqueue ping msg since %s", terrstr());
3!
2619
      rpcFreeCont(rpcMsg.pCont);
3✔
2620
      goto _out;
3✔
2621
    }
2622

2623
  _out:
44,306✔
2624
    if (taosTmrReset(syncNodeEqPingTimer, pNode->pingTimerMS, (void*)pNode->rid, syncEnv()->pTimerManager,
44,309!
2625
                     &pNode->pPingTimer))
2626
      sError("failed to reset ping timer");
×
2627
  }
2628
  syncNodeRelease(pNode);
44,309✔
2629
}
2630

2631
static void syncNodeEqElectTimer(void* param, void* tmrId) {
968✔
2632
  if (!syncIsInit()) return;
972!
2633

2634
  int64_t    rid = (int64_t)param;
968✔
2635
  SSyncNode* pNode = syncNodeAcquire(rid);
968✔
2636

2637
  if (pNode == NULL) return;
968✔
2638

2639
  if (pNode->syncEqMsg == NULL) {
967!
2640
    syncNodeRelease(pNode);
×
2641
    return;
×
2642
  }
2643

2644
  int64_t tsNow = taosGetTimestampMs();
967✔
2645
  if (tsNow < pNode->electTimerParam.executeTime) {
967✔
2646
    syncNodeRelease(pNode);
3✔
2647
    return;
3✔
2648
  }
2649

2650
  SRpcMsg rpcMsg = {0};
964✔
2651
  int32_t code =
2652
      syncBuildTimeout(&rpcMsg, SYNC_TIMEOUT_ELECTION, pNode->electTimerParam.logicClock, pNode->electTimerMS, pNode);
964✔
2653

2654
  if (code != 0) {
964!
2655
    sError("failed to build elect msg");
×
2656
    syncNodeRelease(pNode);
×
2657
    return;
×
2658
  }
2659

2660
  SyncTimeout* pTimeout = rpcMsg.pCont;
964✔
2661
  sNTrace(pNode, "enqueue elect msg lc:%" PRId64, pTimeout->logicClock);
964!
2662

2663
  code = pNode->syncEqMsg(pNode->msgcb, &rpcMsg);
964✔
2664
  if (code != 0) {
964!
2665
    sError("failed to sync enqueue elect msg since %s", terrstr());
×
2666
    rpcFreeCont(rpcMsg.pCont);
×
2667
    syncNodeRelease(pNode);
×
2668
    return;
×
2669
  }
2670

2671
  syncNodeRelease(pNode);
964✔
2672
}
2673

2674
#ifdef BUILD_NO_CALL
2675
static void syncNodeEqHeartbeatTimer(void* param, void* tmrId) {
2676
  if (!syncIsInit()) return;
2677

2678
  int64_t    rid = (int64_t)param;
2679
  SSyncNode* pNode = syncNodeAcquire(rid);
2680

2681
  if (pNode == NULL) return;
2682

2683
  if (pNode->totalReplicaNum > 1) {
2684
    if (atomic_load_64(&pNode->heartbeatTimerLogicClockUser) <= atomic_load_64(&pNode->heartbeatTimerLogicClock)) {
2685
      SRpcMsg rpcMsg = {0};
2686
      int32_t code = syncBuildTimeout(&rpcMsg, SYNC_TIMEOUT_HEARTBEAT, atomic_load_64(&pNode->heartbeatTimerLogicClock),
2687
                                      pNode->heartbeatTimerMS, pNode);
2688

2689
      if (code != 0) {
2690
        sError("failed to build heartbeat msg");
2691
        goto _out;
2692
      }
2693

2694
      sTrace("vgId:%d, enqueue heartbeat timer", pNode->vgId);
2695
      code = pNode->syncEqMsg(pNode->msgcb, &rpcMsg);
2696
      if (code != 0) {
2697
        sError("failed to enqueue heartbeat msg since %s", terrstr());
2698
        rpcFreeCont(rpcMsg.pCont);
2699
        goto _out;
2700
      }
2701

2702
    _out:
2703
      if (taosTmrReset(syncNodeEqHeartbeatTimer, pNode->heartbeatTimerMS, (void*)pNode->rid, syncEnv()->pTimerManager,
2704
                       &pNode->pHeartbeatTimer) != 0)
2705
        return;
2706

2707
    } else {
2708
      sTrace("==syncNodeEqHeartbeatTimer== heartbeatTimerLogicClock:%" PRId64 ", heartbeatTimerLogicClockUser:%" PRId64,
2709
             pNode->heartbeatTimerLogicClock, pNode->heartbeatTimerLogicClockUser);
2710
    }
2711
  }
2712
}
2713
#endif
2714

2715
static void syncNodeEqPeerHeartbeatTimer(void* param, void* tmrId) {
31,427✔
2716
  int32_t code = 0;
31,427✔
2717
  int64_t hbDataRid = (int64_t)param;
31,427✔
2718
  int64_t tsNow = taosGetTimestampMs();
31,427✔
2719

2720
  SSyncHbTimerData* pData = syncHbTimerDataAcquire(hbDataRid);
31,427✔
2721
  if (pData == NULL) {
31,427!
2722
    sError("hb timer get pData NULL, %" PRId64, hbDataRid);
×
2723
    return;
×
2724
  }
2725

2726
  SSyncNode* pSyncNode = syncNodeAcquire(pData->syncNodeRid);
31,427✔
2727
  if (pSyncNode == NULL) {
31,427✔
2728
    syncHbTimerDataRelease(pData);
4✔
2729
    sError("hb timer get pSyncNode NULL");
4!
2730
    return;
4✔
2731
  }
2732

2733
  SSyncTimer* pSyncTimer = pData->pTimer;
31,423✔
2734

2735
  if (!pSyncNode->isStart) {
31,423!
2736
    syncNodeRelease(pSyncNode);
×
2737
    syncHbTimerDataRelease(pData);
×
2738
    sError("vgId:%d, hb timer sync node already stop", pSyncNode->vgId);
×
2739
    return;
×
2740
  }
2741

2742
  if (pSyncNode->state != TAOS_SYNC_STATE_LEADER && pSyncNode->state != TAOS_SYNC_STATE_ASSIGNED_LEADER) {
31,423!
2743
    syncNodeRelease(pSyncNode);
×
2744
    syncHbTimerDataRelease(pData);
×
2745
    sError("vgId:%d, hb timer sync node not leader", pSyncNode->vgId);
×
2746
    return;
×
2747
  }
2748

2749
  sTrace("vgId:%d, peer hb timer execution, rid:%" PRId64 " addr:%" PRId64, pSyncNode->vgId, hbDataRid,
31,423!
2750
         pData->destId.addr);
2751

2752
  if (pSyncNode->totalReplicaNum > 1) {
31,423✔
2753
    int64_t timerLogicClock = atomic_load_64(&pSyncTimer->logicClock);
31,421✔
2754
    int64_t msgLogicClock = atomic_load_64(&pData->logicClock);
31,421✔
2755

2756
    if (timerLogicClock == msgLogicClock) {
31,421✔
2757
      if (tsNow > pData->execTime) {
31,417✔
2758
        pData->execTime += pSyncTimer->timerMS;
31,402✔
2759

2760
        SRpcMsg rpcMsg = {0};
31,402✔
2761
        if ((code = syncBuildHeartbeat(&rpcMsg, pSyncNode->vgId)) != 0) {
31,402!
2762
          sError("vgId:%d, failed to build heartbeat msg since %s", pSyncNode->vgId, tstrerror(code));
×
2763
          syncNodeRelease(pSyncNode);
×
2764
          syncHbTimerDataRelease(pData);
×
2765
          return;
×
2766
        }
2767

2768
        pSyncNode->minMatchIndex = syncMinMatchIndex(pSyncNode);
31,402✔
2769

2770
        SyncHeartbeat* pSyncMsg = rpcMsg.pCont;
31,402✔
2771
        pSyncMsg->srcId = pSyncNode->myRaftId;
31,402✔
2772
        pSyncMsg->destId = pData->destId;
31,402✔
2773
        pSyncMsg->term = raftStoreGetTerm(pSyncNode);
31,402✔
2774
        pSyncMsg->commitIndex = pSyncNode->commitIndex;
31,402✔
2775
        pSyncMsg->minMatchIndex = pSyncNode->minMatchIndex;
31,402✔
2776
        pSyncMsg->privateTerm = 0;
31,402✔
2777
        pSyncMsg->timeStamp = tsNow;
31,402✔
2778

2779
        // update reset time
2780
        int64_t timerElapsed = tsNow - pSyncTimer->timeStamp;
31,402✔
2781
        pSyncTimer->timeStamp = tsNow;
31,402✔
2782

2783
        // send msg
2784
        TRACE_SET_MSGID(&(rpcMsg.info.traceId), tGenIdPI64());
31,402✔
2785
        STraceId* trace = &(rpcMsg.info.traceId);
31,402✔
2786
        sGTrace("vgId:%d, send sync-heartbeat to dnode:%d", pSyncNode->vgId, DID(&(pSyncMsg->destId)));
31,402!
2787
        syncLogSendHeartbeat(pSyncNode, pSyncMsg, false, timerElapsed, pData->execTime);
31,402✔
2788
        int ret = syncNodeSendHeartbeat(pSyncNode, &pSyncMsg->destId, &rpcMsg);
31,402✔
2789
        if (ret != 0) {
31,402✔
2790
          sError("vgId:%d, failed to send heartbeat since %s", pSyncNode->vgId, tstrerror(ret));
46!
2791
        }
2792
      }
2793

2794
      if (syncIsInit()) {
31,417!
2795
        sTrace("vgId:%d, reset peer hb timer at %d", pSyncNode->vgId, pSyncTimer->timerMS);
31,417!
2796
        bool stopped = taosTmrReset(syncNodeEqPeerHeartbeatTimer, pSyncTimer->timerMS, (void*)hbDataRid,
31,417✔
2797
                                    syncEnv()->pTimerManager, &pSyncTimer->pTimer);
31,417✔
2798
        if (stopped) sError("vgId:%d, reset peer hb timer error, %s", pSyncNode->vgId, tstrerror(code));
31,417!
2799

2800
      } else {
2801
        sError("sync env is stop, reset peer hb timer error");
×
2802
      }
2803

2804
    } else {
2805
      sTrace("vgId:%d, do not send hb, timerLogicClock:%" PRId64 ", msgLogicClock:%" PRId64 "", pSyncNode->vgId,
4!
2806
             timerLogicClock, msgLogicClock);
2807
    }
2808
  }
2809

2810
  syncHbTimerDataRelease(pData);
31,423✔
2811
  syncNodeRelease(pSyncNode);
31,423✔
2812
}
2813

2814
#ifdef BUILD_NO_CALL
2815
static void deleteCacheEntry(const void* key, size_t keyLen, void* value, void* ud) {
2816
  (void)ud;
2817
  taosMemoryFree(value);
2818
}
2819

2820
int32_t syncCacheEntry(SSyncLogStore* pLogStore, SSyncRaftEntry* pEntry, LRUHandle** h) {
2821
  SSyncLogStoreData* pData = pLogStore->data;
2822
  sNTrace(pData->pSyncNode, "in cache index:%" PRId64 ", bytes:%u, %p", pEntry->index, pEntry->bytes, pEntry);
2823

2824
  int32_t   code = 0;
2825
  int32_t   entryLen = sizeof(*pEntry) + pEntry->dataLen;
2826
  LRUStatus status = taosLRUCacheInsert(pLogStore->pCache, &pEntry->index, sizeof(pEntry->index), pEntry, entryLen,
2827
                                        deleteCacheEntry, h, TAOS_LRU_PRIORITY_LOW, NULL);
2828
  if (status != TAOS_LRU_STATUS_OK) {
2829
    code = -1;
2830
  }
2831

2832
  return code;
2833
}
2834
#endif
2835

2836
void syncBuildConfigFromReq(SAlterVnodeReplicaReq* pReq, SSyncCfg* cfg) {  // TODO SAlterVnodeReplicaReq name is proper?
×
2837
  cfg->replicaNum = 0;
×
2838
  cfg->totalReplicaNum = 0;
×
2839
  int32_t code = 0;
×
2840

2841
  for (int i = 0; i < pReq->replica; ++i) {
×
2842
    SNodeInfo* pNode = &cfg->nodeInfo[i];
×
2843
    pNode->nodeId = pReq->replicas[i].id;
×
2844
    pNode->nodePort = pReq->replicas[i].port;
×
2845
    tstrncpy(pNode->nodeFqdn, pReq->replicas[i].fqdn, sizeof(pNode->nodeFqdn));
×
2846
    pNode->nodeRole = TAOS_SYNC_ROLE_VOTER;
×
2847
    bool update = tmsgUpdateDnodeInfo(&pNode->nodeId, &pNode->clusterId, pNode->nodeFqdn, &pNode->nodePort);
×
2848
    sInfo("vgId:%d, replica:%d ep:%s:%u dnode:%d nodeRole:%d, update:%d", pReq->vgId, i, pNode->nodeFqdn,
×
2849
          pNode->nodePort, pNode->nodeId, pNode->nodeRole, update);
2850
    cfg->replicaNum++;
×
2851
  }
2852
  if (pReq->selfIndex != -1) {
×
2853
    cfg->myIndex = pReq->selfIndex;
×
2854
  }
2855
  for (int i = cfg->replicaNum; i < pReq->replica + pReq->learnerReplica; ++i) {
×
2856
    SNodeInfo* pNode = &cfg->nodeInfo[i];
×
2857
    pNode->nodeId = pReq->learnerReplicas[cfg->totalReplicaNum].id;
×
2858
    pNode->nodePort = pReq->learnerReplicas[cfg->totalReplicaNum].port;
×
2859
    pNode->nodeRole = TAOS_SYNC_ROLE_LEARNER;
×
2860
    tstrncpy(pNode->nodeFqdn, pReq->learnerReplicas[cfg->totalReplicaNum].fqdn, sizeof(pNode->nodeFqdn));
×
2861
    bool update = tmsgUpdateDnodeInfo(&pNode->nodeId, &pNode->clusterId, pNode->nodeFqdn, &pNode->nodePort);
×
2862
    sInfo("vgId:%d, replica:%d ep:%s:%u dnode:%d nodeRole:%d, update:%d", pReq->vgId, i, pNode->nodeFqdn,
×
2863
          pNode->nodePort, pNode->nodeId, pNode->nodeRole, update);
2864
    cfg->totalReplicaNum++;
×
2865
  }
2866
  cfg->totalReplicaNum += pReq->replica;
×
2867
  if (pReq->learnerSelfIndex != -1) {
×
2868
    cfg->myIndex = pReq->replica + pReq->learnerSelfIndex;
×
2869
  }
2870
  cfg->changeVersion = pReq->changeVersion;
×
2871
}
×
2872

2873
int32_t syncNodeCheckChangeConfig(SSyncNode* ths, SSyncRaftEntry* pEntry) {
×
2874
  int32_t code = 0;
×
2875
  if (pEntry->originalRpcType != TDMT_SYNC_CONFIG_CHANGE) {
×
2876
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
2877
  }
2878

2879
  SMsgHead* head = (SMsgHead*)pEntry->data;
×
2880
  void*     pReq = POINTER_SHIFT(head, sizeof(SMsgHead));
×
2881

2882
  SAlterVnodeTypeReq req = {0};
×
2883
  if (tDeserializeSAlterVnodeReplicaReq(pReq, head->contLen, &req) != 0) {
×
2884
    code = TSDB_CODE_INVALID_MSG;
×
2885
    TAOS_RETURN(code);
×
2886
  }
2887

2888
  SSyncCfg cfg = {0};
×
2889
  syncBuildConfigFromReq(&req, &cfg);
×
2890

2891
  if (cfg.totalReplicaNum >= 1 &&
×
2892
      (ths->state == TAOS_SYNC_STATE_LEADER || ths->state == TAOS_SYNC_STATE_ASSIGNED_LEADER)) {
×
2893
    bool incfg = false;
×
2894
    for (int32_t j = 0; j < cfg.totalReplicaNum; ++j) {
×
2895
      if (strcmp(ths->myNodeInfo.nodeFqdn, cfg.nodeInfo[j].nodeFqdn) == 0 &&
×
2896
          ths->myNodeInfo.nodePort == cfg.nodeInfo[j].nodePort) {
×
2897
        incfg = true;
×
2898
        break;
×
2899
      }
2900
    }
2901

2902
    if (!incfg) {
×
2903
      SyncTerm currentTerm = raftStoreGetTerm(ths);
×
2904
      SRaftId  id = EMPTY_RAFT_ID;
×
2905
      syncNodeStepDown(ths, currentTerm, id);
×
2906
      return 1;
×
2907
    }
2908
  }
2909
  return 0;
×
2910
}
2911

2912
void syncNodeLogConfigInfo(SSyncNode* ths, SSyncCfg* cfg, char* str) {
×
2913
  sInfo("vgId:%d, %s. SyncNode, replicaNum:%d, peersNum:%d, lastConfigIndex:%" PRId64
×
2914
        ", changeVersion:%d, "
2915
        "restoreFinish:%d",
2916
        ths->vgId, str, ths->replicaNum, ths->peersNum, ths->raftCfg.lastConfigIndex, ths->raftCfg.cfg.changeVersion,
2917
        ths->restoreFinish);
2918

2919
  sInfo("vgId:%d, %s, myNodeInfo, clusterId:%" PRId64 ", nodeId:%d, Fqdn:%s, port:%d, role:%d", ths->vgId, str,
×
2920
        ths->myNodeInfo.clusterId, ths->myNodeInfo.nodeId, ths->myNodeInfo.nodeFqdn, ths->myNodeInfo.nodePort,
2921
        ths->myNodeInfo.nodeRole);
2922

2923
  for (int32_t i = 0; i < ths->peersNum; ++i) {
×
2924
    sInfo("vgId:%d, %s, peersNodeInfo%d, clusterId:%" PRId64 ", nodeId:%d, Fqdn:%s, port:%d, role:%d", ths->vgId, str,
×
2925
          i, ths->peersNodeInfo[i].clusterId, ths->peersNodeInfo[i].nodeId, ths->peersNodeInfo[i].nodeFqdn,
2926
          ths->peersNodeInfo[i].nodePort, ths->peersNodeInfo[i].nodeRole);
2927
  }
2928

2929
  for (int32_t i = 0; i < ths->peersNum; ++i) {
×
2930
    char    buf[256];
2931
    int32_t len = 256;
×
2932
    int32_t n = 0;
×
2933
    n += tsnprintf(buf + n, len - n, "%s", "{");
×
2934
    for (int i = 0; i < ths->peersEpset->numOfEps; i++) {
×
2935
      n += tsnprintf(buf + n, len - n, "%s:%d%s", ths->peersEpset->eps[i].fqdn, ths->peersEpset->eps[i].port,
×
2936
                     (i + 1 < ths->peersEpset->numOfEps ? ", " : ""));
×
2937
    }
2938
    n += tsnprintf(buf + n, len - n, "%s", "}");
×
2939

2940
    sInfo("vgId:%d, %s, peersEpset%d, %s, inUse:%d", ths->vgId, str, i, buf, ths->peersEpset->inUse);
×
2941
  }
2942

2943
  for (int32_t i = 0; i < ths->peersNum; ++i) {
×
2944
    sInfo("vgId:%d, %s, peersId%d, addr:%" PRId64, ths->vgId, str, i, ths->peersId[i].addr);
×
2945
  }
2946

2947
  for (int32_t i = 0; i < ths->raftCfg.cfg.totalReplicaNum; ++i) {
×
2948
    sInfo("vgId:%d, %s, nodeInfo%d, clusterId:%" PRId64 ", nodeId:%d, Fqdn:%s, port:%d, role:%d", ths->vgId, str, i,
×
2949
          ths->raftCfg.cfg.nodeInfo[i].clusterId, ths->raftCfg.cfg.nodeInfo[i].nodeId,
2950
          ths->raftCfg.cfg.nodeInfo[i].nodeFqdn, ths->raftCfg.cfg.nodeInfo[i].nodePort,
2951
          ths->raftCfg.cfg.nodeInfo[i].nodeRole);
2952
  }
2953

2954
  for (int32_t i = 0; i < ths->raftCfg.cfg.totalReplicaNum; ++i) {
×
2955
    sInfo("vgId:%d, %s, replicasId%d, addr:%" PRId64, ths->vgId, str, i, ths->replicasId[i].addr);
×
2956
  }
2957
}
×
2958

2959
int32_t syncNodeRebuildPeerAndCfg(SSyncNode* ths, SSyncCfg* cfg) {
×
2960
  int32_t i = 0;
×
2961

2962
  // change peersNodeInfo
2963
  i = 0;
×
2964
  for (int32_t j = 0; j < cfg->totalReplicaNum; ++j) {
×
2965
    if (!(strcmp(ths->myNodeInfo.nodeFqdn, cfg->nodeInfo[j].nodeFqdn) == 0 &&
×
2966
          ths->myNodeInfo.nodePort == cfg->nodeInfo[j].nodePort)) {
×
2967
      ths->peersNodeInfo[i].nodeRole = cfg->nodeInfo[j].nodeRole;
×
2968
      ths->peersNodeInfo[i].clusterId = cfg->nodeInfo[j].clusterId;
×
2969
      tstrncpy(ths->peersNodeInfo[i].nodeFqdn, cfg->nodeInfo[j].nodeFqdn, TSDB_FQDN_LEN);
×
2970
      ths->peersNodeInfo[i].nodeId = cfg->nodeInfo[j].nodeId;
×
2971
      ths->peersNodeInfo[i].nodePort = cfg->nodeInfo[j].nodePort;
×
2972

2973
      syncUtilNodeInfo2EpSet(&ths->peersNodeInfo[i], &ths->peersEpset[i]);
×
2974

2975
      if (syncUtilNodeInfo2RaftId(&ths->peersNodeInfo[i], ths->vgId, &ths->peersId[i]) == false) {
×
2976
        sError("vgId:%d, failed to determine raft member id, peer:%d", ths->vgId, i);
×
2977
        return terrno;
×
2978
      }
2979

2980
      i++;
×
2981
    }
2982
  }
2983
  ths->peersNum = i;
×
2984

2985
  // change cfg nodeInfo
2986
  ths->raftCfg.cfg.replicaNum = 0;
×
2987
  i = 0;
×
2988
  for (int32_t j = 0; j < cfg->totalReplicaNum; ++j) {
×
2989
    if (cfg->nodeInfo[j].nodeRole == TAOS_SYNC_ROLE_VOTER) {
×
2990
      ths->raftCfg.cfg.replicaNum++;
×
2991
    }
2992
    ths->raftCfg.cfg.nodeInfo[i].nodeRole = cfg->nodeInfo[j].nodeRole;
×
2993
    ths->raftCfg.cfg.nodeInfo[i].clusterId = cfg->nodeInfo[j].clusterId;
×
2994
    tstrncpy(ths->raftCfg.cfg.nodeInfo[i].nodeFqdn, cfg->nodeInfo[j].nodeFqdn, TSDB_FQDN_LEN);
×
2995
    ths->raftCfg.cfg.nodeInfo[i].nodeId = cfg->nodeInfo[j].nodeId;
×
2996
    ths->raftCfg.cfg.nodeInfo[i].nodePort = cfg->nodeInfo[j].nodePort;
×
2997
    if ((strcmp(ths->myNodeInfo.nodeFqdn, cfg->nodeInfo[j].nodeFqdn) == 0 &&
×
2998
         ths->myNodeInfo.nodePort == cfg->nodeInfo[j].nodePort)) {
×
2999
      ths->raftCfg.cfg.myIndex = i;
×
3000
    }
3001
    i++;
×
3002
  }
3003
  ths->raftCfg.cfg.totalReplicaNum = i;
×
3004

3005
  return 0;
×
3006
}
3007

3008
void syncNodeChangePeerAndCfgToVoter(SSyncNode* ths, SSyncCfg* cfg) {
×
3009
  // change peersNodeInfo
3010
  for (int32_t i = 0; i < ths->peersNum; ++i) {
×
3011
    for (int32_t j = 0; j < cfg->totalReplicaNum; ++j) {
×
3012
      if (strcmp(ths->peersNodeInfo[i].nodeFqdn, cfg->nodeInfo[j].nodeFqdn) == 0 &&
×
3013
          ths->peersNodeInfo[i].nodePort == cfg->nodeInfo[j].nodePort) {
×
3014
        if (cfg->nodeInfo[j].nodeRole == TAOS_SYNC_ROLE_VOTER) {
×
3015
          ths->peersNodeInfo[i].nodeRole = TAOS_SYNC_ROLE_VOTER;
×
3016
        }
3017
      }
3018
    }
3019
  }
3020

3021
  // change cfg nodeInfo
3022
  ths->raftCfg.cfg.replicaNum = 0;
×
3023
  for (int32_t i = 0; i < ths->raftCfg.cfg.totalReplicaNum; ++i) {
×
3024
    for (int32_t j = 0; j < cfg->totalReplicaNum; ++j) {
×
3025
      if (strcmp(ths->raftCfg.cfg.nodeInfo[i].nodeFqdn, cfg->nodeInfo[j].nodeFqdn) == 0 &&
×
3026
          ths->raftCfg.cfg.nodeInfo[i].nodePort == cfg->nodeInfo[j].nodePort) {
×
3027
        if (cfg->nodeInfo[j].nodeRole == TAOS_SYNC_ROLE_VOTER) {
×
3028
          ths->raftCfg.cfg.nodeInfo[i].nodeRole = TAOS_SYNC_ROLE_VOTER;
×
3029
          ths->raftCfg.cfg.replicaNum++;
×
3030
        }
3031
      }
3032
    }
3033
  }
3034
}
×
3035

3036
int32_t syncNodeRebuildAndCopyIfExist(SSyncNode* ths, int32_t oldtotalReplicaNum) {
×
3037
  int32_t code = 0;
×
3038
  // 1.rebuild replicasId, remove deleted one
3039
  SRaftId oldReplicasId[TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA];
3040
  memcpy(oldReplicasId, ths->replicasId, sizeof(oldReplicasId));
×
3041

3042
  ths->replicaNum = ths->raftCfg.cfg.replicaNum;
×
3043
  ths->totalReplicaNum = ths->raftCfg.cfg.totalReplicaNum;
×
3044
  for (int32_t i = 0; i < ths->raftCfg.cfg.totalReplicaNum; ++i) {
×
3045
    if (syncUtilNodeInfo2RaftId(&ths->raftCfg.cfg.nodeInfo[i], ths->vgId, &ths->replicasId[i]) == false) return terrno;
×
3046
  }
3047

3048
  // 2.rebuild MatchIndex, remove deleted one
3049
  SSyncIndexMgr* oldIndex = ths->pMatchIndex;
×
3050

3051
  ths->pMatchIndex = syncIndexMgrCreate(ths);
×
3052
  if (ths->pMatchIndex == NULL) {
×
3053
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
3054
    if (terrno != 0) code = terrno;
×
3055
    TAOS_RETURN(code);
×
3056
  }
3057

3058
  syncIndexMgrCopyIfExist(ths->pMatchIndex, oldIndex, oldReplicasId);
×
3059

3060
  syncIndexMgrDestroy(oldIndex);
×
3061

3062
  // 3.rebuild NextIndex, remove deleted one
3063
  SSyncIndexMgr* oldNextIndex = ths->pNextIndex;
×
3064

3065
  ths->pNextIndex = syncIndexMgrCreate(ths);
×
3066
  if (ths->pNextIndex == NULL) {
×
3067
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
3068
    if (terrno != 0) code = terrno;
×
3069
    TAOS_RETURN(code);
×
3070
  }
3071

3072
  syncIndexMgrCopyIfExist(ths->pNextIndex, oldNextIndex, oldReplicasId);
×
3073

3074
  syncIndexMgrDestroy(oldNextIndex);
×
3075

3076
  // 4.rebuild pVotesGranted, pVotesRespond, no need to keep old vote state, only rebuild
3077
  voteGrantedUpdate(ths->pVotesGranted, ths);
×
3078
  votesRespondUpdate(ths->pVotesRespond, ths);
×
3079

3080
  // 5.rebuild logReplMgr
3081
  for (int i = 0; i < oldtotalReplicaNum; ++i) {
×
3082
    sDebug("vgId:%d, old logReplMgrs i:%d, peerId:%d, restoreed:%d, [%" PRId64 " %" PRId64 ", %" PRId64 ")", ths->vgId,
×
3083
           i, ths->logReplMgrs[i]->peerId, ths->logReplMgrs[i]->restored, ths->logReplMgrs[i]->startIndex,
3084
           ths->logReplMgrs[i]->matchIndex, ths->logReplMgrs[i]->endIndex);
3085
  }
3086

3087
  SSyncLogReplMgr* oldLogReplMgrs = NULL;
×
3088
  int64_t          length = sizeof(SSyncLogReplMgr) * (TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA);
×
3089
  oldLogReplMgrs = taosMemoryMalloc(length);
×
3090
  if (NULL == oldLogReplMgrs) return terrno;
×
3091
  memset(oldLogReplMgrs, 0, length);
×
3092

3093
  for (int i = 0; i < oldtotalReplicaNum; i++) {
×
3094
    oldLogReplMgrs[i] = *(ths->logReplMgrs[i]);
×
3095
  }
3096

3097
  syncNodeLogReplDestroy(ths);
×
3098
  if ((code = syncNodeLogReplInit(ths)) != 0) {
×
3099
    taosMemoryFree(oldLogReplMgrs);
×
3100
    TAOS_RETURN(code);
×
3101
  }
3102

3103
  for (int i = 0; i < ths->totalReplicaNum; ++i) {
×
3104
    for (int j = 0; j < oldtotalReplicaNum; j++) {
×
3105
      if (syncUtilSameId(&ths->replicasId[i], &oldReplicasId[j])) {
×
3106
        *(ths->logReplMgrs[i]) = oldLogReplMgrs[j];
×
3107
        ths->logReplMgrs[i]->peerId = i;
×
3108
      }
3109
    }
3110
  }
3111

3112
  for (int i = 0; i < ths->totalReplicaNum; ++i) {
×
3113
    sDebug("vgId:%d, new logReplMgrs i:%d, peerId:%d, restoreed:%d, [%" PRId64 " %" PRId64 ", %" PRId64 ")", ths->vgId,
×
3114
           i, ths->logReplMgrs[i]->peerId, ths->logReplMgrs[i]->restored, ths->logReplMgrs[i]->startIndex,
3115
           ths->logReplMgrs[i]->matchIndex, ths->logReplMgrs[i]->endIndex);
3116
  }
3117

3118
  // 6.rebuild sender
3119
  for (int i = 0; i < oldtotalReplicaNum; ++i) {
×
3120
    sDebug("vgId:%d, old sender i:%d, replicaIndex:%d, lastSendTime:%" PRId64, ths->vgId, i,
×
3121
           ths->senders[i]->replicaIndex, ths->senders[i]->lastSendTime)
3122
  }
3123

3124
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
×
3125
    if (ths->senders[i] != NULL) {
×
3126
      sDebug("vgId:%d, snapshot sender destroy while close, data:%p", ths->vgId, ths->senders[i]);
×
3127

3128
      if (snapshotSenderIsStart(ths->senders[i])) {
×
3129
        snapshotSenderStop(ths->senders[i], false);
×
3130
      }
3131

3132
      snapshotSenderDestroy(ths->senders[i]);
×
3133
      ths->senders[i] = NULL;
×
3134
    }
3135
  }
3136

3137
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
×
3138
    SSyncSnapshotSender* pSender = NULL;
×
3139
    int32_t              code = snapshotSenderCreate(ths, i, &pSender);
×
3140
    if (pSender == NULL) return terrno = code;
×
3141

3142
    ths->senders[i] = pSender;
×
3143
    sSDebug(pSender, "snapshot sender create while open sync node, data:%p", pSender);
×
3144
  }
3145

3146
  for (int i = 0; i < ths->totalReplicaNum; i++) {
×
3147
    sDebug("vgId:%d, new sender i:%d, replicaIndex:%d, lastSendTime:%" PRId64, ths->vgId, i,
×
3148
           ths->senders[i]->replicaIndex, ths->senders[i]->lastSendTime)
3149
  }
3150

3151
  // 7.rebuild synctimer
3152
  if ((code = syncNodeStopHeartbeatTimer(ths)) != 0) {
×
3153
    taosMemoryFree(oldLogReplMgrs);
×
3154
    TAOS_RETURN(code);
×
3155
  }
3156

3157
  for (int32_t i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; ++i) {
×
3158
    if ((code = syncHbTimerInit(ths, &ths->peerHeartbeatTimerArr[i], ths->replicasId[i])) != 0) {
×
3159
      taosMemoryFree(oldLogReplMgrs);
×
3160
      TAOS_RETURN(code);
×
3161
    }
3162
  }
3163

3164
  if ((code = syncNodeStartHeartbeatTimer(ths)) != 0) {
×
3165
    taosMemoryFree(oldLogReplMgrs);
×
3166
    TAOS_RETURN(code);
×
3167
  }
3168

3169
  // 8.rebuild peerStates
3170
  SPeerState oldState[TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA] = {0};
×
3171
  for (int i = 0; i < TSDB_MAX_REPLICA + TSDB_MAX_LEARNER_REPLICA; i++) {
×
3172
    oldState[i] = ths->peerStates[i];
×
3173
  }
3174

3175
  for (int i = 0; i < ths->totalReplicaNum; i++) {
×
3176
    for (int j = 0; j < oldtotalReplicaNum; j++) {
×
3177
      if (syncUtilSameId(&ths->replicasId[i], &oldReplicasId[j])) {
×
3178
        ths->peerStates[i] = oldState[j];
×
3179
      }
3180
    }
3181
  }
3182

3183
  taosMemoryFree(oldLogReplMgrs);
×
3184

3185
  return 0;
×
3186
}
3187

3188
void syncNodeChangeToVoter(SSyncNode* ths) {
×
3189
  // replicasId, only need to change replicaNum when 1->3
3190
  ths->replicaNum = ths->raftCfg.cfg.replicaNum;
×
3191
  sDebug("vgId:%d, totalReplicaNum:%d", ths->vgId, ths->totalReplicaNum);
×
3192
  for (int32_t i = 0; i < ths->totalReplicaNum; ++i) {
×
3193
    sDebug("vgId:%d, i:%d, replicaId.addr:%" PRIx64, ths->vgId, i, ths->replicasId[i].addr);
×
3194
  }
3195

3196
  // pMatchIndex, pNextIndex, only need to change replicaNum when 1->3
3197
  ths->pMatchIndex->replicaNum = ths->raftCfg.cfg.replicaNum;
×
3198
  ths->pNextIndex->replicaNum = ths->raftCfg.cfg.replicaNum;
×
3199

3200
  sDebug("vgId:%d, pMatchIndex->totalReplicaNum:%d", ths->vgId, ths->pMatchIndex->totalReplicaNum);
×
3201
  for (int32_t i = 0; i < ths->pMatchIndex->totalReplicaNum; ++i) {
×
3202
    sDebug("vgId:%d, i:%d, match.index:%" PRId64, ths->vgId, i, ths->pMatchIndex->index[i]);
×
3203
  }
3204

3205
  // pVotesGranted, pVotesRespond
3206
  voteGrantedUpdate(ths->pVotesGranted, ths);
×
3207
  votesRespondUpdate(ths->pVotesRespond, ths);
×
3208

3209
  // logRepMgrs
3210
  // no need to change logRepMgrs when 1->3
3211
}
×
3212

3213
void syncNodeResetPeerAndCfg(SSyncNode* ths) {
×
3214
  SNodeInfo node = {0};
×
3215
  for (int32_t i = 0; i < ths->peersNum; ++i) {
×
3216
    memcpy(&ths->peersNodeInfo[i], &node, sizeof(SNodeInfo));
×
3217
  }
3218

3219
  for (int32_t i = 0; i < ths->raftCfg.cfg.totalReplicaNum; ++i) {
×
3220
    memcpy(&ths->raftCfg.cfg.nodeInfo[i], &node, sizeof(SNodeInfo));
×
3221
  }
3222
}
×
3223

3224
int32_t syncNodeChangeConfig(SSyncNode* ths, SSyncRaftEntry* pEntry, char* str) {
×
3225
  int32_t code = 0;
×
3226
  if (pEntry->originalRpcType != TDMT_SYNC_CONFIG_CHANGE) {
×
3227
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
3228
  }
3229

3230
  SMsgHead* head = (SMsgHead*)pEntry->data;
×
3231
  void*     pReq = POINTER_SHIFT(head, sizeof(SMsgHead));
×
3232

3233
  SAlterVnodeTypeReq req = {0};
×
3234
  if (tDeserializeSAlterVnodeReplicaReq(pReq, head->contLen, &req) != 0) {
×
3235
    code = TSDB_CODE_INVALID_MSG;
×
3236
    TAOS_RETURN(code);
×
3237
  }
3238

3239
  SSyncCfg cfg = {0};
×
3240
  syncBuildConfigFromReq(&req, &cfg);
×
3241

3242
  if (cfg.changeVersion <= ths->raftCfg.cfg.changeVersion) {
×
3243
    sInfo(
×
3244
        "vgId:%d, skip conf change entry since lower version. "
3245
        "this entry, index:%" PRId64 ", term:%" PRId64
3246
        ", totalReplicaNum:%d, changeVersion:%d; "
3247
        "current node, replicaNum:%d, peersNum:%d, lastConfigIndex:%" PRId64 ", changeVersion:%d",
3248
        ths->vgId, pEntry->index, pEntry->term, cfg.totalReplicaNum, cfg.changeVersion, ths->replicaNum, ths->peersNum,
3249
        ths->raftCfg.lastConfigIndex, ths->raftCfg.cfg.changeVersion);
3250
    return 0;
×
3251
  }
3252

3253
  if (strcmp(str, "Commit") == 0) {
×
3254
    sInfo(
×
3255
        "vgId:%d, change config from %s. "
3256
        "this, i:%" PRId64
3257
        ", trNum:%d, vers:%d; "
3258
        "node, rNum:%d, pNum:%d, trNum:%d, "
3259
        "buffer: [%" PRId64 " %" PRId64 " %" PRId64 ", %" PRId64
3260
        "), "
3261
        "cond:(next i:%" PRId64 ", t:%" PRId64 " ==%s)",
3262
        ths->vgId, str, pEntry->index - 1, cfg.totalReplicaNum, cfg.changeVersion, ths->replicaNum, ths->peersNum,
3263
        ths->totalReplicaNum, ths->pLogBuf->startIndex, ths->pLogBuf->commitIndex, ths->pLogBuf->matchIndex,
3264
        ths->pLogBuf->endIndex, pEntry->index, pEntry->term, TMSG_INFO(pEntry->originalRpcType));
3265
  } else {
3266
    sInfo(
×
3267
        "vgId:%d, change config from %s. "
3268
        "this, i:%" PRId64 ", t:%" PRId64
3269
        ", trNum:%d, vers:%d; "
3270
        "node, rNum:%d, pNum:%d, trNum:%d, "
3271
        "buffer: [%" PRId64 " %" PRId64 " %" PRId64 ", %" PRId64
3272
        "), "
3273
        "cond:(pre i:%" PRId64 "==ci:%" PRId64 ", bci:%" PRId64 ")",
3274
        ths->vgId, str, pEntry->index, pEntry->term, cfg.totalReplicaNum, cfg.changeVersion, ths->replicaNum,
3275
        ths->peersNum, ths->totalReplicaNum, ths->pLogBuf->startIndex, ths->pLogBuf->commitIndex,
3276
        ths->pLogBuf->matchIndex, ths->pLogBuf->endIndex, pEntry->index - 1, ths->commitIndex,
3277
        ths->pLogBuf->commitIndex);
3278
  }
3279

3280
  syncNodeLogConfigInfo(ths, &cfg, "before config change");
×
3281

3282
  int32_t oldTotalReplicaNum = ths->totalReplicaNum;
×
3283

3284
  if (cfg.totalReplicaNum == 1 || cfg.totalReplicaNum == 2) {  // remove replica
×
3285

3286
    bool incfg = false;
×
3287
    for (int32_t j = 0; j < cfg.totalReplicaNum; ++j) {
×
3288
      if (strcmp(ths->myNodeInfo.nodeFqdn, cfg.nodeInfo[j].nodeFqdn) == 0 &&
×
3289
          ths->myNodeInfo.nodePort == cfg.nodeInfo[j].nodePort) {
×
3290
        incfg = true;
×
3291
        break;
×
3292
      }
3293
    }
3294

3295
    if (incfg) {  // remove other
×
3296
      syncNodeResetPeerAndCfg(ths);
×
3297

3298
      // no need to change myNodeInfo
3299

3300
      if ((code = syncNodeRebuildPeerAndCfg(ths, &cfg)) != 0) {
×
3301
        TAOS_RETURN(code);
×
3302
      };
3303

3304
      if ((code = syncNodeRebuildAndCopyIfExist(ths, oldTotalReplicaNum)) != 0) {
×
3305
        TAOS_RETURN(code);
×
3306
      };
3307
    } else {  // remove myself
3308
      // no need to do anything actually, to change the following to reduce distruptive server chance
3309

3310
      syncNodeResetPeerAndCfg(ths);
×
3311

3312
      // change myNodeInfo
3313
      ths->myNodeInfo.nodeRole = TAOS_SYNC_ROLE_LEARNER;
×
3314

3315
      // change peer and cfg
3316
      ths->peersNum = 0;
×
3317
      memcpy(&ths->raftCfg.cfg.nodeInfo[0], &ths->myNodeInfo, sizeof(SNodeInfo));
×
3318
      ths->raftCfg.cfg.replicaNum = 0;
×
3319
      ths->raftCfg.cfg.totalReplicaNum = 1;
×
3320

3321
      // change other
3322
      if ((code = syncNodeRebuildAndCopyIfExist(ths, oldTotalReplicaNum)) != 0) {
×
3323
        TAOS_RETURN(code);
×
3324
      }
3325

3326
      // change state
3327
      ths->state = TAOS_SYNC_STATE_LEARNER;
×
3328
    }
3329

3330
    ths->restoreFinish = false;
×
3331
  } else {                            // add replica, or change replica type
3332
    if (ths->totalReplicaNum == 3) {  // change replica type
×
3333
      sInfo("vgId:%d, begin change replica type", ths->vgId);
×
3334

3335
      // change myNodeInfo
3336
      for (int32_t j = 0; j < cfg.totalReplicaNum; ++j) {
×
3337
        if (strcmp(ths->myNodeInfo.nodeFqdn, cfg.nodeInfo[j].nodeFqdn) == 0 &&
×
3338
            ths->myNodeInfo.nodePort == cfg.nodeInfo[j].nodePort) {
×
3339
          if (cfg.nodeInfo[j].nodeRole == TAOS_SYNC_ROLE_VOTER) {
×
3340
            ths->myNodeInfo.nodeRole = TAOS_SYNC_ROLE_VOTER;
×
3341
          }
3342
        }
3343
      }
3344

3345
      // change peer and cfg
3346
      syncNodeChangePeerAndCfgToVoter(ths, &cfg);
×
3347

3348
      // change other
3349
      syncNodeChangeToVoter(ths);
×
3350

3351
      // change state
3352
      if (ths->state == TAOS_SYNC_STATE_LEARNER) {
×
3353
        if (ths->myNodeInfo.nodeRole == TAOS_SYNC_ROLE_VOTER) {
×
3354
          ths->state = TAOS_SYNC_STATE_FOLLOWER;
×
3355
        }
3356
      }
3357

3358
      ths->restoreFinish = false;
×
3359
    } else {  // add replica
3360
      sInfo("vgId:%d, begin add replica", ths->vgId);
×
3361

3362
      // no need to change myNodeInfo
3363

3364
      // change peer and cfg
3365
      if ((code = syncNodeRebuildPeerAndCfg(ths, &cfg)) != 0) {
×
3366
        TAOS_RETURN(code);
×
3367
      };
3368

3369
      // change other
3370
      if ((code = syncNodeRebuildAndCopyIfExist(ths, oldTotalReplicaNum)) != 0) {
×
3371
        TAOS_RETURN(code);
×
3372
      };
3373

3374
      // no need to change state
3375

3376
      if (ths->myNodeInfo.nodeRole == TAOS_SYNC_ROLE_LEARNER) {
×
3377
        ths->restoreFinish = false;
×
3378
      }
3379
    }
3380
  }
3381

3382
  ths->quorum = syncUtilQuorum(ths->replicaNum);
×
3383

3384
  ths->raftCfg.lastConfigIndex = pEntry->index;
×
3385
  ths->raftCfg.cfg.lastIndex = pEntry->index;
×
3386
  ths->raftCfg.cfg.changeVersion = cfg.changeVersion;
×
3387

3388
  syncNodeLogConfigInfo(ths, &cfg, "after config change");
×
3389

3390
  if ((code = syncWriteCfgFile(ths)) != 0) {
×
3391
    sError("vgId:%d, failed to create sync cfg file", ths->vgId);
×
3392
    TAOS_RETURN(code);
×
3393
  };
3394

3395
  TAOS_RETURN(code);
×
3396
}
3397

3398
int32_t syncNodeAppend(SSyncNode* ths, SSyncRaftEntry* pEntry) {
9,703,785✔
3399
  int32_t code = -1;
9,703,785✔
3400
  if (pEntry->dataLen < sizeof(SMsgHead)) {
9,703,785!
3401
    code = TSDB_CODE_SYN_INTERNAL_ERROR;
×
3402
    sError("vgId:%d, cannot append an invalid client request with no msg head. type:%s, dataLen:%d", ths->vgId,
×
3403
           TMSG_INFO(pEntry->originalRpcType), pEntry->dataLen);
3404
    syncEntryDestroy(pEntry);
×
3405
    pEntry = NULL;
×
3406
    goto _out;
×
3407
  }
3408

3409
  // append to log buffer
3410
  if ((code = syncLogBufferAppend(ths->pLogBuf, ths, pEntry)) < 0) {
9,703,785✔
3411
    sError("vgId:%d, failed to enqueue sync log buffer, index:%" PRId64, ths->vgId, pEntry->index);
199!
3412
    int32_t ret = 0;
199✔
3413
    if ((ret = syncFsmExecute(ths, ths->pFsm, ths->state, raftStoreGetTerm(ths), pEntry, terrno, false)) != 0) {
199!
UNCOV
3414
      sError("vgId:%d, failed to execute fsm, since %s", ths->vgId, tstrerror(ret));
×
3415
    }
UNCOV
3416
    syncEntryDestroy(pEntry);
×
UNCOV
3417
    pEntry = NULL;
×
UNCOV
3418
    goto _out;
×
3419
  }
3420

3421
  code = 0;
9,703,732✔
3422
_out:;
9,703,732✔
3423
  // proceed match index, with replicating on needed
3424
  SyncIndex matchIndex = syncLogBufferProceed(ths->pLogBuf, ths, NULL, "Append");
9,703,732✔
3425

3426
  if (pEntry != NULL)
9,703,745!
3427
    sTrace("vgId:%d, append raft entry. index:%" PRId64 ", term:%" PRId64 " pBuf: [%" PRId64 " %" PRId64 " %" PRId64
9,703,821✔
3428
           ", %" PRId64 ")",
3429
           ths->vgId, pEntry->index, pEntry->term, ths->pLogBuf->startIndex, ths->pLogBuf->commitIndex,
3430
           ths->pLogBuf->matchIndex, ths->pLogBuf->endIndex);
3431

3432
  if (code == 0 && ths->state == TAOS_SYNC_STATE_ASSIGNED_LEADER) {
9,703,641!
3433
    TAOS_CHECK_RETURN(syncNodeUpdateAssignedCommitIndex(ths, matchIndex));
×
3434

3435
    if (ths->fsmState != SYNC_FSM_STATE_INCOMPLETE &&
×
3436
        syncLogBufferCommit(ths->pLogBuf, ths, ths->assignedCommitIndex) < 0) {
×
3437
      sError("vgId:%d, failed to commit until commitIndex:%" PRId64 "", ths->vgId, ths->commitIndex);
×
3438
      code = TSDB_CODE_SYN_INTERNAL_ERROR;
×
3439
    }
3440
  }
3441

3442
  // multi replica
3443
  if (ths->replicaNum > 1) {
9,703,641✔
3444
    TAOS_RETURN(code);
139,967✔
3445
  }
3446

3447
  // single replica
3448
  SyncIndex returnIndex = syncNodeUpdateCommitIndex(ths, matchIndex);
9,563,674✔
3449
  sTrace("vgId:%d, update commit return index %" PRId64 "", ths->vgId, returnIndex);
9,563,904✔
3450

3451
  if (ths->fsmState != SYNC_FSM_STATE_INCOMPLETE &&
19,127,893!
3452
      (code = syncLogBufferCommit(ths->pLogBuf, ths, ths->commitIndex)) < 0) {
9,563,932✔
3453
    sError("vgId:%d, failed to commit until commitIndex:%" PRId64 "", ths->vgId, ths->commitIndex);
1!
3454
  }
3455

3456
  TAOS_RETURN(code);
9,563,961✔
3457
}
3458

3459
bool syncNodeHeartbeatReplyTimeout(SSyncNode* pSyncNode) {
9,695,050✔
3460
  if (pSyncNode->totalReplicaNum == 1) {
9,695,050✔
3461
    return false;
9,426,189✔
3462
  }
3463

3464
  int32_t toCount = 0;
268,861✔
3465
  int64_t tsNow = taosGetTimestampMs();
268,989✔
3466
  for (int32_t i = 0; i < pSyncNode->peersNum; ++i) {
676,099✔
3467
    if (pSyncNode->peersNodeInfo[i].nodeRole == TAOS_SYNC_ROLE_LEARNER) {
407,126✔
3468
      continue;
128,990✔
3469
    }
3470
    int64_t recvTime = syncIndexMgrGetRecvTime(pSyncNode->pMatchIndex, &(pSyncNode->peersId[i]));
278,136✔
3471
    if (recvTime == 0 || recvTime == -1) {
278,120!
3472
      continue;
×
3473
    }
3474

3475
    if (tsNow - recvTime > tsHeartbeatTimeout) {
278,121✔
3476
      toCount++;
9,768✔
3477
    }
3478
  }
3479

3480
  bool b = (toCount >= pSyncNode->quorum ? true : false);
268,973✔
3481

3482
  return b;
268,973✔
3483
}
3484

3485
bool syncNodeSnapshotSending(SSyncNode* pSyncNode) {
×
3486
  if (pSyncNode == NULL) return false;
×
3487
  bool b = false;
×
3488
  for (int32_t i = 0; i < pSyncNode->totalReplicaNum; ++i) {
×
3489
    if (pSyncNode->senders[i] != NULL && pSyncNode->senders[i]->start) {
×
3490
      b = true;
×
3491
      break;
×
3492
    }
3493
  }
3494
  return b;
×
3495
}
3496

3497
bool syncNodeSnapshotRecving(SSyncNode* pSyncNode) {
×
3498
  if (pSyncNode == NULL) return false;
×
3499
  if (pSyncNode->pNewNodeReceiver == NULL) return false;
×
3500
  if (pSyncNode->pNewNodeReceiver->start) return true;
×
3501
  return false;
×
3502
}
3503

3504
static int32_t syncNodeAppendNoop(SSyncNode* ths) {
9,799✔
3505
  int32_t   code = 0;
9,799✔
3506
  SyncIndex index = syncLogBufferGetEndIndex(ths->pLogBuf);
9,799✔
3507
  SyncTerm  term = raftStoreGetTerm(ths);
9,799✔
3508

3509
  SSyncRaftEntry* pEntry = syncEntryBuildNoop(term, index, ths->vgId);
9,799✔
3510
  if (pEntry == NULL) {
9,799!
3511
    code = TSDB_CODE_OUT_OF_MEMORY;
×
3512
    TAOS_RETURN(code);
×
3513
  }
3514

3515
  code = syncNodeAppend(ths, pEntry);
9,799✔
3516
  TAOS_RETURN(code);
9,799✔
3517
}
3518

3519
#ifdef BUILD_NO_CALL
3520
static int32_t syncNodeAppendNoopOld(SSyncNode* ths) {
3521
  int32_t ret = 0;
3522

3523
  SyncIndex       index = ths->pLogStore->syncLogWriteIndex(ths->pLogStore);
3524
  SyncTerm        term = raftStoreGetTerm(ths);
3525
  SSyncRaftEntry* pEntry = syncEntryBuildNoop(term, index, ths->vgId);
3526
  if (pEntry == NULL) return -1;
3527

3528
  LRUHandle* h = NULL;
3529

3530
  if (ths->state == TAOS_SYNC_STATE_LEADER) {
3531
    int32_t code = ths->pLogStore->syncLogAppendEntry(ths->pLogStore, pEntry, false);
3532
    if (code != 0) {
3533
      sError("append noop error");
3534
      return -1;
3535
    }
3536

3537
    syncCacheEntry(ths->pLogStore, pEntry, &h);
3538
  }
3539

3540
  if (h) {
3541
    taosLRUCacheRelease(ths->pLogStore->pCache, h, false);
3542
  } else {
3543
    syncEntryDestroy(pEntry);
3544
  }
3545

3546
  return ret;
3547
}
3548
#endif
3549

3550
int32_t syncNodeOnHeartbeat(SSyncNode* ths, const SRpcMsg* pRpcMsg) {
30,440✔
3551
  SyncHeartbeat* pMsg = pRpcMsg->pCont;
30,440✔
3552
  bool           resetElect = false;
30,440✔
3553

3554
  const STraceId* trace = &pRpcMsg->info.traceId;
30,440✔
3555
  char            tbuf[40] = {0};
30,440✔
3556
  TRACE_TO_STR(trace, tbuf);
30,440!
3557

3558
  int64_t tsMs = taosGetTimestampMs();
30,440✔
3559
  int64_t timeDiff = tsMs - pMsg->timeStamp;
30,440✔
3560
  syncLogRecvHeartbeat(ths, pMsg, timeDiff, tbuf);
30,440✔
3561

3562
  if (!syncNodeInRaftGroup(ths, &pMsg->srcId)) {
30,440✔
3563
    sWarn(
5!
3564
        "vgId:%d, drop heartbeat msg from dnode:%d, because it come from another cluster:%d, differ from current "
3565
        "cluster:%d",
3566
        ths->vgId, DID(&(pMsg->srcId)), CID(&(pMsg->srcId)), CID(&(ths->myRaftId)));
3567
    return 0;
5✔
3568
  }
3569

3570
  SRpcMsg rpcMsg = {0};
30,435✔
3571
  TAOS_CHECK_RETURN(syncBuildHeartbeatReply(&rpcMsg, ths->vgId));
30,435!
3572
  SyncTerm currentTerm = raftStoreGetTerm(ths);
30,435✔
3573

3574
  SyncHeartbeatReply* pMsgReply = rpcMsg.pCont;
30,435✔
3575
  pMsgReply->destId = pMsg->srcId;
30,435✔
3576
  pMsgReply->srcId = ths->myRaftId;
30,435✔
3577
  pMsgReply->term = currentTerm;
30,435✔
3578
  pMsgReply->privateTerm = 8864;  // magic number
30,435✔
3579
  pMsgReply->startTime = ths->startTime;
30,435✔
3580
  pMsgReply->timeStamp = tsMs;
30,435✔
3581

3582
  sGTrace("vgId:%d, process sync-heartbeat msg from dnode:%d, cluster:%d, Msgterm:%" PRId64 " currentTerm:%" PRId64,
30,435!
3583
          ths->vgId, DID(&(pMsg->srcId)), CID(&(pMsg->srcId)), pMsg->term, currentTerm);
3584

3585
  if (pMsg->term > currentTerm && ths->state == TAOS_SYNC_STATE_LEARNER) {
30,435✔
3586
    raftStoreSetTerm(ths, pMsg->term);
239✔
3587
    currentTerm = pMsg->term;
239✔
3588
  }
3589

3590
  if (pMsg->term == currentTerm &&
30,435✔
3591
      (ths->state != TAOS_SYNC_STATE_LEADER && ths->state != TAOS_SYNC_STATE_ASSIGNED_LEADER)) {
30,367!
3592
    syncIndexMgrSetRecvTime(ths->pNextIndex, &(pMsg->srcId), tsMs);
30,367✔
3593
    resetElect = true;
30,367✔
3594

3595
    ths->minMatchIndex = pMsg->minMatchIndex;
30,367✔
3596

3597
    if (ths->state == TAOS_SYNC_STATE_FOLLOWER || ths->state == TAOS_SYNC_STATE_LEARNER) {
30,367✔
3598
      SRpcMsg rpcMsgLocalCmd = {0};
30,364✔
3599
      TAOS_CHECK_RETURN(syncBuildLocalCmd(&rpcMsgLocalCmd, ths->vgId));
30,364!
3600

3601
      SyncLocalCmd* pSyncMsg = rpcMsgLocalCmd.pCont;
30,364✔
3602
      pSyncMsg->cmd =
30,364✔
3603
          (ths->state == TAOS_SYNC_STATE_LEARNER) ? SYNC_LOCAL_CMD_LEARNER_CMT : SYNC_LOCAL_CMD_FOLLOWER_CMT;
30,364✔
3604
      pSyncMsg->commitIndex = pMsg->commitIndex;
30,364✔
3605
      pSyncMsg->currentTerm = pMsg->term;
30,364✔
3606

3607
      if (ths->syncEqMsg != NULL && ths->msgcb != NULL) {
30,364!
3608
        int32_t code = ths->syncEqMsg(ths->msgcb, &rpcMsgLocalCmd);
30,364✔
3609
        if (code != 0) {
30,364!
3610
          sError("vgId:%d, failed to enqueue commit msg from heartbeat since %s, code:%d", ths->vgId, terrstr(), code);
×
3611
          rpcFreeCont(rpcMsgLocalCmd.pCont);
×
3612
        } else {
3613
          sTrace("vgId:%d, enqueue commit msg from heartbeat, commit-index:%" PRId64 ", term:%" PRId64, ths->vgId,
30,364!
3614
                 pMsg->commitIndex, pMsg->term);
3615
        }
3616
      }
3617
    }
3618
  }
3619

3620
  if (pMsg->term >= currentTerm &&
30,435!
3621
      (ths->state == TAOS_SYNC_STATE_LEADER || ths->state == TAOS_SYNC_STATE_ASSIGNED_LEADER)) {
30,435!
3622
    SRpcMsg rpcMsgLocalCmd = {0};
×
3623
    TAOS_CHECK_RETURN(syncBuildLocalCmd(&rpcMsgLocalCmd, ths->vgId));
×
3624

3625
    SyncLocalCmd* pSyncMsg = rpcMsgLocalCmd.pCont;
×
3626
    pSyncMsg->cmd = SYNC_LOCAL_CMD_STEP_DOWN;
×
3627
    pSyncMsg->currentTerm = pMsg->term;
×
3628
    pSyncMsg->commitIndex = pMsg->commitIndex;
×
3629

3630
    if (ths->syncEqMsg != NULL && ths->msgcb != NULL) {
×
3631
      int32_t code = ths->syncEqMsg(ths->msgcb, &rpcMsgLocalCmd);
×
3632
      if (code != 0) {
×
3633
        sError("vgId:%d, sync enqueue step-down msg error, code:%d", ths->vgId, code);
×
3634
        rpcFreeCont(rpcMsgLocalCmd.pCont);
×
3635
      } else {
3636
        sTrace("vgId:%d, sync enqueue step-down msg, new-term:%" PRId64, ths->vgId, pMsg->term);
×
3637
      }
3638
    }
3639
  }
3640

3641
  // reply
3642
  TAOS_CHECK_RETURN(syncNodeSendMsgById(&pMsgReply->destId, ths, &rpcMsg));
30,435!
3643

3644
  if (resetElect) syncNodeResetElectTimer(ths);
30,435✔
3645
  return 0;
30,435✔
3646
}
3647

3648
int32_t syncNodeOnHeartbeatReply(SSyncNode* ths, const SRpcMsg* pRpcMsg) {
30,223✔
3649
  int32_t         code = 0;
30,223✔
3650
  const STraceId* trace = &pRpcMsg->info.traceId;
30,223✔
3651
  char            tbuf[40] = {0};
30,223✔
3652
  TRACE_TO_STR(trace, tbuf);
30,223!
3653

3654
  SyncHeartbeatReply* pMsg = pRpcMsg->pCont;
30,223✔
3655
  SSyncLogReplMgr*    pMgr = syncNodeGetLogReplMgr(ths, &pMsg->srcId);
30,223✔
3656
  if (pMgr == NULL) {
30,223!
3657
    code = TSDB_CODE_SYN_RETURN_VALUE_NULL;
×
3658
    if (terrno != 0) code = terrno;
×
3659
    sError("vgId:%d, failed to get log repl mgr for the peer at addr 0x016%" PRIx64 "", ths->vgId, pMsg->srcId.addr);
×
3660
    TAOS_RETURN(code);
×
3661
  }
3662

3663
  int64_t tsMs = taosGetTimestampMs();
30,223✔
3664
  syncLogRecvHeartbeatReply(ths, pMsg, tsMs - pMsg->timeStamp, tbuf);
30,223✔
3665

3666
  syncIndexMgrSetRecvTime(ths->pMatchIndex, &pMsg->srcId, tsMs);
30,223✔
3667

3668
  return syncLogReplProcessHeartbeatReply(pMgr, ths, pMsg);
30,222✔
3669
}
3670

3671
#ifdef BUILD_NO_CALL
3672
int32_t syncNodeOnHeartbeatReplyOld(SSyncNode* ths, const SRpcMsg* pRpcMsg) {
3673
  SyncHeartbeatReply* pMsg = pRpcMsg->pCont;
3674

3675
  const STraceId* trace = &pRpcMsg->info.traceId;
3676
  char            tbuf[40] = {0};
3677
  TRACE_TO_STR(trace, tbuf);
3678

3679
  int64_t tsMs = taosGetTimestampMs();
3680
  int64_t timeDiff = tsMs - pMsg->timeStamp;
3681
  syncLogRecvHeartbeatReply(ths, pMsg, timeDiff, tbuf);
3682

3683
  // update last reply time, make decision whether the other node is alive or not
3684
  syncIndexMgrSetRecvTime(ths->pMatchIndex, &pMsg->srcId, tsMs);
3685
  return 0;
3686
}
3687
#endif
3688

3689
int32_t syncNodeOnLocalCmd(SSyncNode* ths, const SRpcMsg* pRpcMsg) {
30,364✔
3690
  SyncLocalCmd* pMsg = pRpcMsg->pCont;
30,364✔
3691
  syncLogRecvLocalCmd(ths, pMsg, "");
30,364✔
3692

3693
  if (pMsg->cmd == SYNC_LOCAL_CMD_STEP_DOWN) {
30,364!
3694
    SRaftId id = EMPTY_RAFT_ID;
×
3695
    syncNodeStepDown(ths, pMsg->currentTerm, id);
×
3696

3697
  } else if (pMsg->cmd == SYNC_LOCAL_CMD_FOLLOWER_CMT || pMsg->cmd == SYNC_LOCAL_CMD_LEARNER_CMT) {
60,728!
3698
    if (syncLogBufferIsEmpty(ths->pLogBuf)) {
30,364!
3699
      sError("vgId:%d, sync log buffer is empty.", ths->vgId);
×
3700
      return 0;
×
3701
    }
3702
    SyncTerm matchTerm = syncLogBufferGetLastMatchTerm(ths->pLogBuf);
30,364✔
3703
    if (matchTerm < 0) {
30,364!
3704
      return TSDB_CODE_SYN_INTERNAL_ERROR;
×
3705
    }
3706
    if (pMsg->currentTerm == matchTerm) {
30,364✔
3707
      SyncIndex returnIndex = syncNodeUpdateCommitIndex(ths, pMsg->commitIndex);
27,563✔
3708
      sTrace("vgId:%d, update commit return index %" PRId64 "", ths->vgId, returnIndex);
27,563!
3709
    }
3710
    if (ths->fsmState != SYNC_FSM_STATE_INCOMPLETE && syncLogBufferCommit(ths->pLogBuf, ths, ths->commitIndex) < 0) {
30,364!
3711
      sError("vgId:%d, failed to commit raft log since %s. commit index:%" PRId64 "", ths->vgId, terrstr(),
×
3712
             ths->commitIndex);
3713
    }
3714
  } else {
3715
    sError("error local cmd");
×
3716
  }
3717

3718
  return 0;
30,364✔
3719
}
3720

3721
// TLA+ Spec
3722
// ClientRequest(i, v) ==
3723
//     /\ state[i] = Leader
3724
//     /\ LET entry == [term  |-> currentTerm[i],
3725
//                      value |-> v]
3726
//            newLog == Append(log[i], entry)
3727
//        IN  log' = [log EXCEPT ![i] = newLog]
3728
//     /\ UNCHANGED <<messages, serverVars, candidateVars,
3729
//                    leaderVars, commitIndex>>
3730
//
3731

3732
int32_t syncNodeOnClientRequest(SSyncNode* ths, SRpcMsg* pMsg, SyncIndex* pRetIndex) {
9,693,979✔
3733
  sNTrace(ths, "on client request");
9,693,979✔
3734

3735
  int32_t code = 0;
9,693,979✔
3736

3737
  SyncIndex       index = syncLogBufferGetEndIndex(ths->pLogBuf);
9,693,979✔
3738
  SyncTerm        term = raftStoreGetTerm(ths);
9,694,158✔
3739
  SSyncRaftEntry* pEntry = NULL;
9,694,148✔
3740
  if (pMsg->msgType == TDMT_SYNC_CLIENT_REQUEST) {
9,694,148✔
3741
    pEntry = syncEntryBuildFromClientRequest(pMsg->pCont, term, index);
205,264✔
3742
  } else {
3743
    pEntry = syncEntryBuildFromRpcMsg(pMsg, term, index);
9,488,884✔
3744
  }
3745

3746
  if (pEntry == NULL) {
9,694,113!
3747
    sError("vgId:%d, failed to process client request since %s.", ths->vgId, terrstr());
×
3748
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
3749
  }
3750

3751
  // 1->2, config change is add in write thread, and will continue in sync thread
3752
  // need save message for it
3753
  if (pMsg->msgType == TDMT_SYNC_CONFIG_CHANGE) {
9,694,113!
3754
    SRespStub stub = {.createTime = taosGetTimestampMs(), .rpcMsg = *pMsg};
×
3755
    uint64_t  seqNum = syncRespMgrAdd(ths->pSyncRespMgr, &stub);
×
3756
    pEntry->seqNum = seqNum;
×
3757
  }
3758

3759
  if (ths->state == TAOS_SYNC_STATE_LEADER || ths->state == TAOS_SYNC_STATE_ASSIGNED_LEADER) {
9,694,113!
3760
    if (pRetIndex) {
9,694,113✔
3761
      (*pRetIndex) = index;
9,488,814✔
3762
    }
3763

3764
    if (pEntry->originalRpcType == TDMT_SYNC_CONFIG_CHANGE) {
9,694,113!
3765
      int32_t code = syncNodeCheckChangeConfig(ths, pEntry);
×
3766
      if (code < 0) {
×
3767
        sError("vgId:%d, failed to check change config since %s.", ths->vgId, terrstr());
×
3768
        syncEntryDestroy(pEntry);
×
3769
        pEntry = NULL;
×
3770
        TAOS_RETURN(code);
×
3771
      }
3772

3773
      if (code > 0) {
×
3774
        SRpcMsg rsp = {.code = pMsg->code, .info = pMsg->info};
×
3775
        int32_t num = syncRespMgrGetAndDel(ths->pSyncRespMgr, pEntry->seqNum, &rsp.info);
×
3776
        sDebug("vgId:%d, get response stub for config change, seqNum:%" PRIu64 ", num:%d", ths->vgId, pEntry->seqNum,
×
3777
               num);
3778
        if (rsp.info.handle != NULL) {
×
3779
          tmsgSendRsp(&rsp);
×
3780
        }
3781
        syncEntryDestroy(pEntry);
×
3782
        pEntry = NULL;
×
3783
        TAOS_RETURN(code);
×
3784
      }
3785
    }
3786

3787
    code = syncNodeAppend(ths, pEntry);
9,694,113✔
3788
    return code;
9,693,835✔
3789
  } else {
3790
    syncEntryDestroy(pEntry);
×
3791
    pEntry = NULL;
×
3792
    return TSDB_CODE_SYN_INTERNAL_ERROR;
×
3793
  }
3794
}
3795

3796
const char* syncStr(ESyncState state) {
346,280✔
3797
  switch (state) {
346,280!
3798
    case TAOS_SYNC_STATE_FOLLOWER:
102,562✔
3799
      return "follower";
102,562✔
3800
    case TAOS_SYNC_STATE_CANDIDATE:
2,175✔
3801
      return "candidate";
2,175✔
3802
    case TAOS_SYNC_STATE_LEADER:
232,497✔
3803
      return "leader";
232,497✔
3804
    case TAOS_SYNC_STATE_ERROR:
×
3805
      return "error";
×
3806
    case TAOS_SYNC_STATE_OFFLINE:
2,843✔
3807
      return "offline";
2,843✔
3808
    case TAOS_SYNC_STATE_LEARNER:
6,201✔
3809
      return "learner";
6,201✔
3810
    case TAOS_SYNC_STATE_ASSIGNED_LEADER:
×
3811
      return "assigned leader";
×
3812
    default:
2✔
3813
      return "unknown";
2✔
3814
  }
3815
}
3816

3817
int32_t syncNodeUpdateNewConfigIndex(SSyncNode* ths, SSyncCfg* pNewCfg) {
1,421✔
3818
  for (int32_t i = 0; i < pNewCfg->totalReplicaNum; ++i) {
1,604!
3819
    SRaftId raftId = {
1,604✔
3820
        .addr = SYNC_ADDR(&pNewCfg->nodeInfo[i]),
1,604✔
3821
        .vgId = ths->vgId,
1,604✔
3822
    };
3823

3824
    if (syncUtilSameId(&(ths->myRaftId), &raftId)) {
1,604✔
3825
      pNewCfg->myIndex = i;
1,421✔
3826
      return 0;
1,421✔
3827
    }
3828
  }
3829

3830
  return TSDB_CODE_SYN_INTERNAL_ERROR;
×
3831
}
3832

3833
bool syncNodeIsOptimizedOneReplica(SSyncNode* ths, SRpcMsg* pMsg) {
9,695,021✔
3834
  return (ths->replicaNum == 1 && syncUtilUserCommit(pMsg->msgType) && ths->vgId != 1);
9,695,021!
3835
}
3836

3837
bool syncNodeInRaftGroup(SSyncNode* ths, SRaftId* pRaftId) {
5,577,099✔
3838
  for (int32_t i = 0; i < ths->totalReplicaNum; ++i) {
8,667,443✔
3839
    if (syncUtilSameId(&((ths->replicasId)[i]), pRaftId)) {
8,667,441✔
3840
      return true;
5,577,092✔
3841
    }
3842
  }
3843
  return false;
2✔
3844
}
3845

3846
SSyncSnapshotSender* syncNodeGetSnapshotSender(SSyncNode* ths, SRaftId* pDestId) {
58,123✔
3847
  SSyncSnapshotSender* pSender = NULL;
58,123✔
3848
  for (int32_t i = 0; i < ths->totalReplicaNum; ++i) {
196,665✔
3849
    if (syncUtilSameId(pDestId, &((ths->replicasId)[i]))) {
138,519✔
3850
      pSender = (ths->senders)[i];
58,153✔
3851
    }
3852
  }
3853
  return pSender;
58,146✔
3854
}
3855

3856
SSyncTimer* syncNodeGetHbTimer(SSyncNode* ths, SRaftId* pDestId) {
21,509✔
3857
  SSyncTimer* pTimer = NULL;
21,509✔
3858
  for (int32_t i = 0; i < ths->totalReplicaNum; ++i) {
91,514✔
3859
    if (syncUtilSameId(pDestId, &((ths->replicasId)[i]))) {
70,005✔
3860
      pTimer = &((ths->peerHeartbeatTimerArr)[i]);
21,509✔
3861
    }
3862
  }
3863
  return pTimer;
21,509✔
3864
}
3865

3866
SPeerState* syncNodeGetPeerState(SSyncNode* ths, const SRaftId* pDestId) {
17,568✔
3867
  SPeerState* pState = NULL;
17,568✔
3868
  for (int32_t i = 0; i < ths->totalReplicaNum; ++i) {
38,127✔
3869
    if (syncUtilSameId(pDestId, &((ths->replicasId)[i]))) {
20,559✔
3870
      pState = &((ths->peerStates)[i]);
17,568✔
3871
    }
3872
  }
3873
  return pState;
17,568✔
3874
}
3875

3876
#ifdef BUILD_NO_CALL
3877
bool syncNodeNeedSendAppendEntries(SSyncNode* ths, const SRaftId* pDestId, const SyncAppendEntries* pMsg) {
3878
  SPeerState* pState = syncNodeGetPeerState(ths, pDestId);
3879
  if (pState == NULL) {
3880
    sError("vgId:%d, replica maybe dropped", ths->vgId);
3881
    return false;
3882
  }
3883

3884
  SyncIndex sendIndex = pMsg->prevLogIndex + 1;
3885
  int64_t   tsNow = taosGetTimestampMs();
3886

3887
  if (pState->lastSendIndex == sendIndex && tsNow - pState->lastSendTime < SYNC_APPEND_ENTRIES_TIMEOUT_MS) {
3888
    return false;
3889
  }
3890

3891
  return true;
3892
}
3893

3894
bool syncNodeCanChange(SSyncNode* pSyncNode) {
3895
  if (pSyncNode->changing) {
3896
    sError("sync cannot change");
3897
    return false;
3898
  }
3899

3900
  if ((pSyncNode->commitIndex >= SYNC_INDEX_BEGIN)) {
3901
    SyncIndex lastIndex = syncNodeGetLastIndex(pSyncNode);
3902
    if (pSyncNode->commitIndex != lastIndex) {
3903
      sError("sync cannot change2");
3904
      return false;
3905
    }
3906
  }
3907

3908
  for (int32_t i = 0; i < pSyncNode->peersNum; ++i) {
3909
    SSyncSnapshotSender* pSender = syncNodeGetSnapshotSender(pSyncNode, &(pSyncNode->peersId)[i]);
3910
    if (pSender != NULL && pSender->start) {
3911
      sError("sync cannot change3");
3912
      return false;
3913
    }
3914
  }
3915

3916
  return true;
3917
}
3918
#endif
STATUS · Troubleshooting · Open an Issue · Sales · Support · CAREERS · ENTERPRISE · START FREE · SCHEDULE DEMO
ANNOUNCEMENTS · TWITTER · TOS & SLA · Supported CI Services · What's a CI service? · Automated Testing

© 2026 Coveralls, Inc