Merge pull request #14604 from taosdata/feature/3.0_mhli
refactor(sync): add term in rpcMsg
This commit is contained in:
commit
767d21fa81
|
@ -215,6 +215,7 @@ int32_t syncProposeBatch(int64_t rid, SRpcMsg* pMsgArr, bool* pIsWeakArr, in
|
|||
bool syncEnvIsStart();
|
||||
const char* syncStr(ESyncState state);
|
||||
bool syncIsRestoreFinish(int64_t rid);
|
||||
int32_t syncGetSnapshotByIndex(int64_t rid, SyncIndex index, SSnapshot* pSnapshot);
|
||||
|
||||
int32_t syncReconfig(int64_t rid, const SSyncCfg* pNewCfg);
|
||||
|
||||
|
|
|
@ -35,6 +35,7 @@ typedef struct {
|
|||
uint32_t clientIp;
|
||||
uint16_t clientPort;
|
||||
int64_t applyIndex;
|
||||
uint64_t applyTerm;
|
||||
char user[TSDB_USER_LEN];
|
||||
} SRpcConnInfo;
|
||||
|
||||
|
|
|
@ -234,17 +234,14 @@ int32_t vnodeProcessSyncMsg(SVnode *pVnode, SRpcMsg *pMsg, SRpcMsg **pRsp) {
|
|||
}
|
||||
|
||||
#if 1
|
||||
do {
|
||||
char *syncNodeStr = sync2SimpleStr(pVnode->sync);
|
||||
static int64_t vndTick = 0;
|
||||
if (++vndTick % 10 == 1) {
|
||||
vGTrace("vgId:%d, sync trace msg:%s, %s", syncGetVgId(pVnode->sync), TMSG_INFO(pMsg->msgType), syncNodeStr);
|
||||
}
|
||||
if (gRaftDetailLog) {
|
||||
char logBuf[512] = {0};
|
||||
snprintf(logBuf, sizeof(logBuf), "vnode process syncmsg, msgType:%d, syncNode:%s", pMsg->msgType, syncNodeStr);
|
||||
syncRpcMsgLog2(logBuf, pMsg);
|
||||
}
|
||||
taosMemoryFree(syncNodeStr);
|
||||
} while (0);
|
||||
#endif
|
||||
|
||||
if (syncNodeStrategy(pSyncNode) == SYNC_STRATEGY_NO_SNAPSHOT) {
|
||||
|
@ -297,7 +294,8 @@ int32_t vnodeProcessSyncMsg(SVnode *pVnode, SRpcMsg *pMsg, SRpcMsg **pRsp) {
|
|||
vGError("vgId:%d, msg:%p failed to process since error msg type:%d", pVnode->config.vgId, pMsg->msgType);
|
||||
code = -1;
|
||||
}
|
||||
} else {
|
||||
|
||||
} else if (syncNodeStrategy(pSyncNode) == SYNC_STRATEGY_WAL_FIRST) {
|
||||
// use wal first strategy
|
||||
if (pMsg->msgType == TDMT_SYNC_TIMEOUT) {
|
||||
SyncTimeout *pSyncMsg = syncTimeoutFromRpcMsg2(pMsg);
|
||||
|
@ -404,42 +402,52 @@ static void vnodeSyncReconfig(struct SSyncFSM *pFsm, const SRpcMsg *pMsg, SReCon
|
|||
|
||||
static void vnodeSyncCommitMsg(SSyncFSM *pFsm, const SRpcMsg *pMsg, SFsmCbMeta cbMeta) {
|
||||
SVnode *pVnode = pFsm->data;
|
||||
SSnapshot snapshot = {0};
|
||||
SyncIndex beginIndex = SYNC_INDEX_INVALID;
|
||||
char logBuf[256] = {0};
|
||||
|
||||
snprintf(logBuf, sizeof(logBuf),
|
||||
"commitCb execute, pFsm:%p, index:%ld, isWeak:%d, code:%d, state:%d %s, beginIndex :%ld\n", pFsm,
|
||||
cbMeta.index, cbMeta.isWeak, cbMeta.code, cbMeta.state, syncUtilState2String(cbMeta.state), beginIndex);
|
||||
syncRpcMsgLog2(logBuf, (SRpcMsg *)pMsg);
|
||||
vTrace("vgId:%d, commit-cb is excuted, fsm:%p, index:%ld, isWeak:%d, code:%d, state:%d %s, msgtype:%d %s",
|
||||
syncGetVgId(pVnode->sync), pFsm, cbMeta.index, cbMeta.isWeak, cbMeta.code, cbMeta.state,
|
||||
syncUtilState2String(cbMeta.state), pMsg->msgType, TMSG_INFO(pMsg->msgType));
|
||||
|
||||
SRpcMsg rpcMsg = {.msgType = pMsg->msgType, .contLen = pMsg->contLen};
|
||||
rpcMsg.pCont = rpcMallocCont(rpcMsg.contLen);
|
||||
memcpy(rpcMsg.pCont, pMsg->pCont, pMsg->contLen);
|
||||
syncGetAndDelRespRpc(pVnode->sync, cbMeta.seqNum, &rpcMsg.info);
|
||||
rpcMsg.info.conn.applyIndex = cbMeta.index;
|
||||
rpcMsg.info.conn.applyTerm = cbMeta.term;
|
||||
tmsgPutToQueue(&pVnode->msgCb, APPLY_QUEUE, &rpcMsg);
|
||||
}
|
||||
|
||||
static void vnodeSyncPreCommitMsg(SSyncFSM *pFsm, const SRpcMsg *pMsg, SFsmCbMeta cbMeta) {
|
||||
char logBuf[256] = {0};
|
||||
snprintf(logBuf, sizeof(logBuf), "preCommitCb== pFsm:%p, index:%ld, isWeak:%d, code:%d, state:%d %s \n", pFsm,
|
||||
cbMeta.index, cbMeta.isWeak, cbMeta.code, cbMeta.state, syncUtilState2String(cbMeta.state));
|
||||
syncRpcMsgLog2(logBuf, (SRpcMsg *)pMsg);
|
||||
SVnode *pVnode = pFsm->data;
|
||||
vTrace("vgId:%d, pre-commit-cb is excuted, fsm:%p, index:%ld, isWeak:%d, code:%d, state:%d %s, msgtype:%d %s",
|
||||
syncGetVgId(pVnode->sync), pFsm, cbMeta.index, cbMeta.isWeak, cbMeta.code, cbMeta.state,
|
||||
syncUtilState2String(cbMeta.state), pMsg->msgType, TMSG_INFO(pMsg->msgType));
|
||||
}
|
||||
|
||||
static void vnodeSyncRollBackMsg(SSyncFSM *pFsm, const SRpcMsg *pMsg, SFsmCbMeta cbMeta) {
|
||||
char logBuf[256] = {0};
|
||||
snprintf(logBuf, sizeof(logBuf), "rollBackCb== pFsm:%p, index:%ld, isWeak:%d, code:%d, state:%d %s \n", pFsm,
|
||||
cbMeta.index, cbMeta.isWeak, cbMeta.code, cbMeta.state, syncUtilState2String(cbMeta.state));
|
||||
syncRpcMsgLog2(logBuf, (SRpcMsg *)pMsg);
|
||||
SVnode *pVnode = pFsm->data;
|
||||
vTrace("vgId:%d, rollback-cb is excuted, fsm:%p, index:%ld, isWeak:%d, code:%d, state:%d %s, msgtype:%d %s",
|
||||
syncGetVgId(pVnode->sync), pFsm, cbMeta.index, cbMeta.isWeak, cbMeta.code, cbMeta.state,
|
||||
syncUtilState2String(cbMeta.state), pMsg->msgType, TMSG_INFO(pMsg->msgType));
|
||||
}
|
||||
|
||||
static int32_t vnodeSnapshotStartRead(struct SSyncFSM *pFsm, void *pParam, void **ppReader) { return 0; }
|
||||
static int32_t vnodeSnapshotStartRead(struct SSyncFSM *pFsm, void *pParam, void **ppReader) {
|
||||
SVnode *pVnode = pFsm->data;
|
||||
SSnapshotParam *pSnapshotParam = pParam;
|
||||
int32_t code =
|
||||
vnodeSnapshotReaderOpen(pVnode, (SVSnapshotReader **)ppReader, pSnapshotParam->start, pSnapshotParam->end);
|
||||
return code;
|
||||
}
|
||||
|
||||
static int32_t vnodeSnapshotStopRead(struct SSyncFSM *pFsm, void *pReader) { return 0; }
|
||||
static int32_t vnodeSnapshotStopRead(struct SSyncFSM *pFsm, void *pReader) {
|
||||
SVnode *pVnode = pFsm->data;
|
||||
int32_t code = vnodeSnapshotReaderClose(pReader);
|
||||
return code;
|
||||
}
|
||||
|
||||
static int32_t vnodeSnapshotDoRead(struct SSyncFSM *pFsm, void *pReader, void **ppBuf, int32_t *len) { return 0; }
|
||||
static int32_t vnodeSnapshotDoRead(struct SSyncFSM *pFsm, void *pReader, void **ppBuf, int32_t *len) {
|
||||
SVnode *pVnode = pFsm->data;
|
||||
int32_t code = vnodeSnapshotRead(pReader, (const void **)ppBuf, len);
|
||||
return code;
|
||||
}
|
||||
|
||||
static int32_t vnodeSnapshotStartWrite(struct SSyncFSM *pFsm, void *pParam, void **ppWriter) { return 0; }
|
||||
|
||||
|
|
|
@ -36,6 +36,7 @@ typedef struct SRaftCfg {
|
|||
TdFilePtr pFile;
|
||||
char path[TSDB_FILENAME_LEN * 2];
|
||||
int8_t isStandBy;
|
||||
int32_t batchSize;
|
||||
int8_t snapshotStrategy;
|
||||
SyncIndex lastConfigIndex;
|
||||
|
||||
|
@ -62,6 +63,7 @@ int32_t raftCfgFromStr(const char *s, SRaftCfg *pRaftCfg);
|
|||
|
||||
typedef struct SRaftCfgMeta {
|
||||
int8_t isStandBy;
|
||||
int32_t batchSize;
|
||||
int8_t snapshotStrategy;
|
||||
SyncIndex lastConfigIndex;
|
||||
} SRaftCfgMeta;
|
||||
|
|
|
@ -834,7 +834,7 @@ int32_t syncNodeOnAppendEntriesSnapshot2Cb(SSyncNode* ths, SyncAppendEntriesBatc
|
|||
//
|
||||
// operation:
|
||||
// if hasAppendEntries && pMsg->prevLogIndex == ths->commitIndex, append entry
|
||||
// match my-commit-index or my-commit-index + 1
|
||||
// match my-commit-index or my-commit-index + batchSize
|
||||
do {
|
||||
bool condition = (pMsg->term == ths->pRaftStore->currentTerm) && (ths->state == TAOS_SYNC_STATE_FOLLOWER) &&
|
||||
(pMsg->prevLogIndex <= ths->commitIndex);
|
||||
|
@ -928,11 +928,13 @@ int32_t syncNodeOnAppendEntriesSnapshot2Cb(SSyncNode* ths, SyncAppendEntriesBatc
|
|||
bool condition = condition1 || condition2;
|
||||
|
||||
if (condition) {
|
||||
do {
|
||||
char logBuf[128];
|
||||
snprintf(logBuf, sizeof(logBuf),
|
||||
"recv sync-append-entries-batch, not match, pre-index:%ld, pre-term:%lu, datalen:%d", pMsg->prevLogIndex,
|
||||
pMsg->prevLogTerm, pMsg->dataLen);
|
||||
"recv sync-append-entries-batch, not match, pre-index:%ld, pre-term:%lu, datalen:%d",
|
||||
pMsg->prevLogIndex, pMsg->prevLogTerm, pMsg->dataLen);
|
||||
syncNodeEventLog(ths, logBuf);
|
||||
} while (0);
|
||||
|
||||
// prepare response msg
|
||||
SyncAppendEntriesReply* pReply = syncAppendEntriesReplyBuild(ths->vgId);
|
||||
|
|
|
@ -109,19 +109,30 @@ int32_t syncNodeOnAppendEntriesReplyCb(SSyncNode* ths, SyncAppendEntriesReply* p
|
|||
}
|
||||
|
||||
// only start once
|
||||
static void syncNodeStartSnapshot(SSyncNode* ths, SyncIndex beginIndex, SyncIndex endIndex, SyncTerm lastApplyTerm,
|
||||
static void syncNodeStartSnapshotOnce(SSyncNode* ths, SyncIndex beginIndex, SyncIndex endIndex, SyncTerm lastApplyTerm,
|
||||
SyncAppendEntriesReply* pMsg) {
|
||||
// get sender
|
||||
SSyncSnapshotSender* pSender = syncNodeGetSnapshotSender(ths, &(pMsg->srcId));
|
||||
ASSERT(pSender != NULL);
|
||||
|
||||
if (snapshotSenderIsStart(pSender)) {
|
||||
do {
|
||||
char* eventLog = snapshotSender2SimpleStr(pSender, "snapshot sender already start");
|
||||
syncNodeErrorLog(ths, eventLog);
|
||||
taosMemoryFree(eventLog);
|
||||
} while (0);
|
||||
|
||||
return;
|
||||
}
|
||||
|
||||
SSnapshot snapshot = {
|
||||
.data = NULL, .lastApplyIndex = endIndex, .lastApplyTerm = lastApplyTerm, .lastConfigIndex = SYNC_INDEX_INVALID};
|
||||
|
||||
void* pReader = NULL;
|
||||
SSnapshotParam readerParam = {.start = beginIndex, .end = endIndex};
|
||||
ths->pFsm->FpSnapshotStartRead(ths->pFsm, &readerParam, &pReader);
|
||||
if (!snapshotSenderIsStart(pSender) && pMsg->privateTerm < pSender->privateTerm) {
|
||||
int32_t code = ths->pFsm->FpSnapshotStartRead(ths->pFsm, &readerParam, &pReader);
|
||||
ASSERT(code == 0);
|
||||
|
||||
if (pMsg->privateTerm < pSender->privateTerm) {
|
||||
ASSERT(pReader != NULL);
|
||||
snapshotSenderStart(pSender, readerParam, snapshot, pReader);
|
||||
|
||||
|
@ -178,7 +189,9 @@ int32_t syncNodeOnAppendEntriesReplySnapshot2Cb(SSyncNode* ths, SyncAppendEntrie
|
|||
// start snapshot <match+1, old snapshot.end>
|
||||
SSnapshot oldSnapshot;
|
||||
ths->pFsm->FpGetSnapshotInfo(ths->pFsm, &oldSnapshot);
|
||||
syncNodeStartSnapshot(ths, newMatchIndex + 1, oldSnapshot.lastApplyIndex, oldSnapshot.lastApplyTerm, pMsg);
|
||||
ASSERT(oldSnapshot.lastApplyIndex >= newMatchIndex + 1);
|
||||
syncNodeStartSnapshotOnce(ths, newMatchIndex + 1, oldSnapshot.lastApplyIndex, oldSnapshot.lastApplyTerm,
|
||||
pMsg); // term maybe not ok?
|
||||
|
||||
syncIndexMgrSetIndex(ths->pNextIndex, &(pMsg->srcId), oldSnapshot.lastApplyIndex + 1);
|
||||
syncIndexMgrSetIndex(ths->pMatchIndex, &(pMsg->srcId), newMatchIndex);
|
||||
|
@ -187,7 +200,6 @@ int32_t syncNodeOnAppendEntriesReplySnapshot2Cb(SSyncNode* ths, SyncAppendEntrie
|
|||
} else {
|
||||
SyncIndex nextIndex = syncIndexMgrGetIndex(ths->pNextIndex, &(pMsg->srcId));
|
||||
|
||||
// notice! int64, uint64
|
||||
if (nextIndex > SYNC_INDEX_BEGIN) {
|
||||
--nextIndex;
|
||||
|
||||
|
@ -198,7 +210,7 @@ int32_t syncNodeOnAppendEntriesReplySnapshot2Cb(SSyncNode* ths, SyncAppendEntrie
|
|||
SSyncRaftEntry* pEntry;
|
||||
int32_t code = ths->pLogStore->syncLogGetEntry(ths->pLogStore, nextIndex, &pEntry);
|
||||
ASSERT(code == 0);
|
||||
syncNodeStartSnapshot(ths, SYNC_INDEX_BEGIN, nextIndex, pEntry->term, pMsg);
|
||||
syncNodeStartSnapshotOnce(ths, SYNC_INDEX_BEGIN, nextIndex, pEntry->term, pMsg);
|
||||
|
||||
// get sender
|
||||
SSyncSnapshotSender* pSender = syncNodeGetSnapshotSender(ths, &(pMsg->srcId));
|
||||
|
|
|
@ -397,6 +397,38 @@ bool syncIsRestoreFinish(int64_t rid) {
|
|||
return b;
|
||||
}
|
||||
|
||||
int32_t syncGetSnapshotByIndex(int64_t rid, SyncIndex index, SSnapshot* pSnapshot) {
|
||||
if (index < SYNC_INDEX_BEGIN) {
|
||||
return -1;
|
||||
}
|
||||
|
||||
SSyncNode* pSyncNode = (SSyncNode*)taosAcquireRef(tsNodeRefId, rid);
|
||||
if (pSyncNode == NULL) {
|
||||
return -1;
|
||||
}
|
||||
ASSERT(rid == pSyncNode->rid);
|
||||
|
||||
SSyncRaftEntry* pEntry = NULL;
|
||||
int32_t code = pSyncNode->pLogStore->syncLogGetEntry(pSyncNode->pLogStore, index, &pEntry);
|
||||
if (code != 0) {
|
||||
if (pEntry != NULL) {
|
||||
syncEntryDestory(pEntry);
|
||||
}
|
||||
taosReleaseRef(tsNodeRefId, pSyncNode->rid);
|
||||
return -1;
|
||||
}
|
||||
ASSERT(pEntry != NULL);
|
||||
|
||||
pSnapshot->data = NULL;
|
||||
pSnapshot->lastApplyIndex = index;
|
||||
pSnapshot->lastApplyTerm = pEntry->term;
|
||||
pSnapshot->lastConfigIndex = syncNodeGetSnapshotConfigIndex(pSyncNode, index);
|
||||
|
||||
syncEntryDestory(pEntry);
|
||||
taosReleaseRef(tsNodeRefId, pSyncNode->rid);
|
||||
return 0;
|
||||
}
|
||||
|
||||
int32_t syncGetSnapshotMeta(int64_t rid, struct SSnapshotMeta* sMeta) {
|
||||
SSyncNode* pSyncNode = (SSyncNode*)taosAcquireRef(tsNodeRefId, rid);
|
||||
if (pSyncNode == NULL) {
|
||||
|
@ -786,6 +818,7 @@ int32_t syncNodePropose(SSyncNode* pSyncNode, SRpcMsg* pMsg, bool isWeak) {
|
|||
int32_t code = syncNodeOnClientRequestCb(pSyncNode, pSyncMsg, &retIndex);
|
||||
if (code == 0) {
|
||||
pMsg->info.conn.applyIndex = retIndex;
|
||||
pMsg->info.conn.applyTerm = pSyncNode->pRaftStore->currentTerm;
|
||||
rpcFreeCont(rpcMsg.pCont);
|
||||
syncRespMgrDel(pSyncNode->pSyncRespMgr, seqNum);
|
||||
ret = 1;
|
||||
|
@ -846,6 +879,7 @@ SSyncNode* syncNodeOpen(const SSyncInfo* pOldSyncInfo) {
|
|||
meta.isStandBy = pSyncInfo->isStandBy;
|
||||
meta.snapshotStrategy = pSyncInfo->snapshotStrategy;
|
||||
meta.lastConfigIndex = SYNC_INDEX_INVALID;
|
||||
meta.batchSize = pSyncInfo->batchSize;
|
||||
ret = raftCfgCreateFile((SSyncCfg*)&(pSyncInfo->syncCfg), meta, pSyncNode->configPath);
|
||||
ASSERT(ret == 0);
|
||||
|
||||
|
|
|
@ -183,6 +183,7 @@ cJSON *raftCfg2Json(SRaftCfg *pRaftCfg) {
|
|||
cJSON_AddItemToObject(pRoot, "SSyncCfg", syncCfg2Json(&(pRaftCfg->cfg)));
|
||||
cJSON_AddNumberToObject(pRoot, "isStandBy", pRaftCfg->isStandBy);
|
||||
cJSON_AddNumberToObject(pRoot, "snapshotStrategy", pRaftCfg->snapshotStrategy);
|
||||
cJSON_AddNumberToObject(pRoot, "batchSize", pRaftCfg->batchSize);
|
||||
|
||||
char buf64[128];
|
||||
snprintf(buf64, sizeof(buf64), "%ld", pRaftCfg->lastConfigIndex);
|
||||
|
@ -228,6 +229,7 @@ int32_t raftCfgCreateFile(SSyncCfg *pCfg, SRaftCfgMeta meta, const char *path) {
|
|||
SRaftCfg raftCfg;
|
||||
raftCfg.cfg = *pCfg;
|
||||
raftCfg.isStandBy = meta.isStandBy;
|
||||
raftCfg.batchSize = meta.batchSize;
|
||||
raftCfg.snapshotStrategy = meta.snapshotStrategy;
|
||||
raftCfg.lastConfigIndex = meta.lastConfigIndex;
|
||||
raftCfg.configIndexCount = 1;
|
||||
|
@ -257,6 +259,9 @@ int32_t raftCfgFromJson(const cJSON *pRoot, SRaftCfg *pRaftCfg) {
|
|||
cJSON *pJsonIsStandBy = cJSON_GetObjectItem(pJson, "isStandBy");
|
||||
pRaftCfg->isStandBy = cJSON_GetNumberValue(pJsonIsStandBy);
|
||||
|
||||
cJSON *pJsonBatchSize = cJSON_GetObjectItem(pJson, "batchSize");
|
||||
pRaftCfg->batchSize = cJSON_GetNumberValue(pJsonBatchSize);
|
||||
|
||||
cJSON *pJsonSnapshotStrategy = cJSON_GetObjectItem(pJson, "snapshotStrategy");
|
||||
pRaftCfg->snapshotStrategy = cJSON_GetNumberValue(pJsonSnapshotStrategy);
|
||||
|
||||
|
|
|
@ -8,7 +8,6 @@ void print(SHashObj *pNextIndex) {
|
|||
printf("----------------\n");
|
||||
uint64_t *p = (uint64_t *)taosHashIterate(pNextIndex, NULL);
|
||||
while (p) {
|
||||
|
||||
size_t len;
|
||||
void * key = taosHashGetKey(p, &len);
|
||||
|
||||
|
|
|
@ -26,6 +26,7 @@ SRaftCfg* createRaftCfg() {
|
|||
snprintf(((pCfg->cfg.nodeInfo)[i]).nodeFqdn, sizeof(((pCfg->cfg.nodeInfo)[i]).nodeFqdn), "100.200.300.%d", i);
|
||||
}
|
||||
pCfg->isStandBy = taosGetTimestampSec() % 100;
|
||||
pCfg->batchSize = taosGetTimestampSec() % 100;
|
||||
|
||||
pCfg->configIndexCount = 5;
|
||||
for (int i = 0; i < MAX_CONFIG_INDEX_COUNT; ++i) {
|
||||
|
@ -84,6 +85,7 @@ void test3() {
|
|||
SRaftCfgMeta meta;
|
||||
meta.isStandBy = 7;
|
||||
meta.snapshotStrategy = 9;
|
||||
meta.batchSize = 10;
|
||||
meta.lastConfigIndex = 789;
|
||||
raftCfgCreateFile(pCfg, meta, s);
|
||||
printf("%s create json file: %s \n", (char*)__FUNCTION__, s);
|
||||
|
@ -109,6 +111,7 @@ void test5() {
|
|||
pCfg->cfg.myIndex = taosGetTimestampSec();
|
||||
pCfg->isStandBy += 2;
|
||||
pCfg->snapshotStrategy += 3;
|
||||
pCfg->batchSize += 4;
|
||||
pCfg->lastConfigIndex += 1000;
|
||||
|
||||
pCfg->configIndexCount = 5;
|
||||
|
|
Loading…
Reference in New Issue