More consesus voting / heart beat work

This commit is contained in:
Raycho Mukelov 2023-09-06 19:18:02 +03:00
parent e333d6e7c0
commit 3380c83bde
9 changed files with 110 additions and 83 deletions

View File

@ -7,8 +7,7 @@
# This file may not be copied, modified, or distributed except according to
# those terms.
import asyncdispatch
import std/times
import chronos
template awaitWithTimeout[T](operation: Future[T],
deadline: Future[void],

View File

@ -14,6 +14,16 @@ import protocol
import log_ops
import chronicles
proc RaftNodeQuorumMin[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): bool =
result = false
withLock(node.raftStateMutex):
var cnt = 0
for peer in node.peers:
if peer.hasVoted:
cnt.inc
if cnt >= (node.peers.len div 2 + 1):
result = true
proc RaftNodeStartElection*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) {.async.} =
withLock(node.raftStateMutex):
debug "Raft Node started election. Node ID: ", node_id=node.id
@ -24,58 +34,54 @@ proc RaftNodeStartElection*[SmCommandType, SmStateType](node: RaftNode[SmCommand
for peer in node.peers:
peer.hasVoted = false
node.votesFuts.add(node.msgSendCallback(
RaftMessageRequestVote(lastLogTerm: RaftNodeLogEntryGet(node, RaftNodeLogIndexGet(node)).value.term, lastLogIndex: RaftNodeLogIndexGet(node), senderTerm: node.currentTerm)
RaftMessageRequestVote(
op: rmoRequestVote, msgId: genUUID(), senderId: node.id,
receiverId: peer.id, lastLogTerm: RaftNodeLogEntryGet(node, RaftNodeLogIndexGet(node)).term,
lastLogIndex: RaftNodeLogIndexGet(node), senderTerm: node.currentTerm)
)
)
# Process votes (if any)
for voteFut in node.votesFuts:
var
r: RaftMessageRequestVoteResponse
r = RaftMessageRequestVoteResponse(waitFor voteFut)
let r = await voteFut
let respVote = RaftMessageRequestVoteResponse(r)
debugEcho "r: ", repr(r)
debug "voteFut.finished", voteFut_finished=voteFut.finished
withLock(node.raftStateMutex):
for p in node.peers:
debug "voteFut: ", Response=repr(r)
debug "senderId: ", sender_id=r.senderId
debug "granted: ", granted=r.granted
if p.id == r.senderId:
p.hasVoted = r.granted
debug "senderId: ", sender_id=respVote.senderId
debug "granted: ", granted=respVote.granted
if p.id == respVote.senderId:
p.hasVoted = respVote.granted
withLock(node.raftStateMutex):
node.votesFuts.clear
while node.votesFuts.len > 0:
discard node.votesFuts.pop
proc RaftNodeAbortElection*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
withLock(node.raftStateMutex):
for fut in node.voteFuts:
if not fut.finished and not fut.failed:
cancel(fut)
if node.state == rnsCandidate:
if RaftNodeQuorumMin(node):
node.state = rnsLeader # Transition to leader and send Heart-Beat to establish this node as the cluster leader
RaftNodeSendHeartBeat(node)
else:
asyncSpawn RaftNodeStartElection(node)
proc RaftNodeProcessRequestVote*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], msg: RaftMessageRequestVote): RaftMessageRequestVoteResponse =
proc RaftNodeHandleRequestVote*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], msg: RaftMessageRequestVote): RaftMessageRequestVoteResponse =
withLock(node.raftStateMutex):
result = RaftMessageRequestVoteResponse(msgId: msg.msgId, senderId: msg.senderId, receiverId: msg.reciverId, granted: false)
if msg.senderTerm > node.term:
if msg.lastLogIndex >= RaftNodeLogIndexGet(node) and msg.lastLogTerm >= RaftNodeLogEntryGet(RaftNodeLogIndexGet(node)).term:
result = RaftMessageRequestVoteResponse(msgId: msg.msgId, senderId: node.id, receiverId: msg.senderId, granted: false)
if node.state != rnsCandidate and node.state != rnsStopped and msg.senderTerm > node.currentTerm:
if msg.lastLogIndex >= RaftNodeLogIndexGet(node) and msg.lastLogTerm >= RaftNodeLogEntryGet(node, RaftNodeLogIndexGet(node)).term:
result.granted = true
proc RaftNodeProcessAppendEntries*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], msg: RaftMessageAppendEntries): RaftMessageAppendEntriesResponse =
discard
proc RaftNodeProcessHeartBeat*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], msg: RaftMessageAppendEntries): RaftMessageAppendEntriesResponse =
discard
proc RaftNodeQuorumMin[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): bool =
proc RaftNodeHandleAppendEntries*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], msg: RaftMessageAppendEntries): RaftMessageAppendEntriesResponse[SmStateType] =
discard
proc RaftNodeReplicateSmCommand*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], cmd: SmCommandType) =
discard
proc RaftNodeScheduleRequestVotesCleanUpTimeout*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
discard
proc RaftNodeLogAppend[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], logEntry: RaftNodeLogEntry[SmCommandType]) =
discard

View File

@ -13,5 +13,5 @@ import types
proc RaftNodeLogIndexGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): RaftLogIndex =
discard
proc RaftNodeLogEntryGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], logIndex: RaftLogIndex): Result[RaftNodeLogEntry[SmCommandType], string] =
proc RaftNodeLogEntryGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], logIndex: RaftLogIndex): RaftNodeLogEntry[SmCommandType] =
discard

View File

@ -26,7 +26,9 @@ export
# Forward declarations
proc RaftNodeSmInit[SmCommandType, SmStateType](stateMachine: var RaftNodeStateMachine[SmCommandType, SmStateType])
proc RaftNodeSendHeartBeat*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType])
proc RaftNodeAbortElection*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType])
proc RaftNodeScheduleHeartBeatTimeout*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): Future[void] {.async.}
proc RaftNodeCancelAllTimers[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType])
# Raft Node Public API
proc new*[SmCommandType, SmStateType](T: type RaftNode[SmCommandType, SmStateType]; # Create New Raft Node
@ -41,7 +43,8 @@ proc new*[SmCommandType, SmStateType](T: type RaftNode[SmCommandType, SmStateTyp
result = T(
id: id, state: rnsFollower, currentTerm: 0, peers: peers, commitIndex: 0, lastApplied: 0,
msgSendCallback: msgSendCallback, votedFor: DefaultUUID, currentLeaderId: DefaultUUID
msgSendCallback: msgSendCallback, votedFor: DefaultUUID, currentLeaderId: DefaultUUID,
)
RaftNodeSmInit(result.stateMachine)
@ -53,29 +56,44 @@ proc RaftNodeLoad*[SmCommandType, SmStateType](
discard
proc RaftNodeIdGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): RaftNodeId {.gcsafe.} = # Get Raft Node ID
result = node.id
withLock(node.raftStateMutex):
result = node.id
proc RaftNodeStateGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): RaftNodeState = # Get Raft Node State
node.state
withLock(node.raftStateMutex):
result = node.state
proc RaftNodeTermGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): RaftNodeTerm = # Get Raft Node Term
node.currentTerm
withLock(node.raftStateMutex):
result = node.currentTerm
func RaftNodePeersGet*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): RaftNodePeers = # Get Raft Node Peers
node.peers
withLock(node.raftStateMutex):
result = node.peers
func RaftNodeIsLeader*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): bool = # Check if Raft Node is Leader
node.state == rnsLeader
withLock(node.raftStateMutex):
result = node.state == rnsLeader
# Deliver Raft Message to the Raft Node and dispatch it
proc RaftNodeHandleHeartBeat*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], msg: RaftMessageAppendEntries): RaftMessageAppendEntriesResponse[SmStateType] =
result = RaftMessageAppendEntriesResponse[SmStateType](op: rmoAppendLogEntry, senderId: node.id, receiverId: msg.senderId, msgId: msg.msgId, success: false)
if msg.senderTerm >= node.currentTerm:
result.success = true
RaftNodeCancelAllTimers(node)
RaftNodeAbortElection(node)
proc RaftNodeMessageDeliver*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], raftMessage: RaftMessageBase): Future[RaftMessageResponseBase] {.async, gcsafe.} =
case raftMessage.op
of rmoRequestVote: # Dispatch different Raft Message types based on the operation code
discard
result = RaftNodeHandleRequestVote(node, RaftMessageRequestVote(raftMessage))
of rmoAppendLogEntry:
discard
var appendMsg = RaftMessageAppendEntries[SmCommandType](raftMessage)
if appendMsg.logEntries.isSome:
result = RaftNodeHandleAppendEntries(node, appendMsg)
else:
result = RaftNodeHandleHeartBeat(node, appendMsg)
else: discard
discard
# Process RaftNodeClientRequests
proc RaftNodeServeClientRequest*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType], req: RaftNodeClientRequest[SmCommandType]): Future[RaftNodeClientResponse[SmStateType]] {.async, gcsafe.} =
@ -103,47 +121,55 @@ proc RaftNodeSmApply[SmCommandType, SmStateType](stateMachine: RaftNodeStateMach
mixin RaftSmApply
RaftSmApply(stateMachine, command)
# Private Abstract Timer manipulation Ops
template RaftTimerCreate(timerInterval: int, oneshot: bool, timerCallback: RaftTimerCallback): RaftTimer =
# Private Abstract Timer creation
template RaftTimerCreate(timerInterval: int, timerCallback: RaftTimerCallback): Future[void] =
mixin RaftTimerCreateCustomImpl
RaftTimerCreateCustomImpl(timerInterval, oneshot, timerCallback)
RaftTimerCreateCustomImpl(timerInterval, timerCallback)
# Timers scheduling stuff etc.
proc RaftNodeScheduleHeartBeat*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
var fut = sleepAsync(node.heartBeatTimeout)
fut.callback = proc () = RaftNodeSendHeartBeat(node)
proc RaftNodeScheduleHeartBeatTimeout*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): Future[void] {.async.} =
node.heartBeatTimeoutTimer = sleepAsync(node.heartBeatTimeout)
await node.heartBeatTimeoutTimer
node.state = rnsCandidate # Transition to candidate state and initiate new Election
var f = RaftNodeStartElection(node)
cancel(f)
proc RaftNodeScheduleHeartBeat*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): Future[void] {.async.} =
node.heartBeatTimer = RaftTimerCreate(150, proc() = RaftNodeSendHeartBeat(node))
proc RaftNodeSendHeartBeat*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
for raftPeer in node.peers:
let msgHrtBt = RaftMessageAppendEntries(
senderId: node.id, receiverId: raftPeer.id,
let msgHrtBt = RaftMessageAppendEntries[SmCommandType](
op: rmoAppendLogEntry, senderId: node.id, receiverId: raftPeer.id,
senderTerm: RaftNodeTermGet(node), commitIndex: node.commitIndex,
prevLogIndex: RaftNodeLogIndexGet(node) - 1, prevLogTerm: if RaftNodeLogIndexGet(node) > 0: RaftNodeLogEntry(node, RaftNodeLogIndexGet(node) - 1).term else: 0
prevLogIndex: RaftNodeLogIndexGet(node) - 1, prevLogTerm: if RaftNodeLogIndexGet(node) > 0: RaftNodeLogEntryGet(node, RaftNodeLogIndexGet(node) - 1).term else: 0
)
asyncSpawn node.msgSendCallback(msgHrtBt)
RaftNodeScheduleHeartBeat(node)
discard node.msgSendCallback(msgHrtBt)
asyncSpawn RaftNodeScheduleHeartBeat(node)
proc RaftNodeScheduleHeartBeatTimeout*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]): Future[void] {.async.} =
node.heartBeatTimeoutTimer = RaftTimerCreate(180, proc =
withLock(node.raftStateMutex):
node.state = rnsCandidate # Transition to candidate state and initiate new Election
asyncSpawn RaftNodeStartElection(node)
)
# Raft Node Control
proc RaftNodeAbortElection*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
withLock(node.raftStateMutex):
node.state = rnsFollower
for fut in node.votesFuts:
waitFor cancelAndWait(fut)
asyncSpawn RaftNodeScheduleHeartBeatTimeout(node)
proc RaftNodeCancelAllTimers[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
node.requestVotesTimer.fail(newException(Exception, "fail"))
node.heartBeatTimer.fail(newException(Exception, "fail"))
node.heartBeatTimeoutTimer.fail(newException(Exception, "fail"))
node.appendEntriesTimer.fail(newException(Exception, "fail"))
withLock(node.raftStateMutex):
waitFor cancelAndWait(node.requestVotesTimer)
waitFor cancelAndWait(node.heartBeatTimer)
waitFor cancelAndWait(node.heartBeatTimeoutTimer)
waitFor cancelAndWait(node.appendEntriesTimer)
proc RaftNodeStop*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
# Try to stop gracefully
node.state = rnsStopped
withLock(node.raftStateMutex):
node.state = rnsStopped
# Cancel pending timers (if any)
var f = RaftNodeCancelAllTimers(node)
RaftNodeCancelAllTimers(node)
proc RaftNodeStart*[SmCommandType, SmStateType](node: RaftNode[SmCommandType, SmStateType]) =
debug "Start Raft Node with ID: ", nodeid=node.id
node.state = rnsFollower
asyncSpawn RaftNodeScheduleHeartBeatTimeout(node)
debug "Start Raft Node with ID: ", nodeid=node.id

View File

@ -119,14 +119,6 @@ type
# Probably this will be a RocksDB/MDBX/SQLite Store Wrapper etc.
logData*: seq[RaftNodeLogEntry[SmCommandType]] # Raft Node Log Data
# Timer types
RaftTimer* = ref object
mtx*: Lock
canceled*: bool
expired*: bool
timeout*: int
oneshot*: bool
RaftTimerCallback* = proc () {.gcsafe.} # Pass any function wrapped in a closure
# Raft Node Object type

View File

@ -28,7 +28,7 @@ proc BasicRaftClusterStart*(cluster: BasicRaftCluster) =
for id, node in cluster.nodes:
RaftNodeStart(node)
proc BasicRaftClusterGetLeader*(cluster: BasicRaftCluster): UUID =
proc BasicRaftClusterGetLeaderId*(cluster: BasicRaftCluster): UUID =
result = DefaultUUID
for id, node in cluster.nodes:
if RaftNodeIsLeader(node):
@ -38,8 +38,10 @@ proc BasicRaftClusterClientRequest*(cluster: BasicRaftCluster, req: RaftNodeClie
case req.op:
of rncroRequestSmState:
var
nodeId = cluster.nodesIds[random(cluster.nodesIds.len)]
result =
nodeId = cluster.nodesIds[BasicRaftClusterGetLeaderId(cluster)]
result = await cluster.nodes[nodeId].RaftNodeServeClientRequest(req)
of rncroExecSmCommand:
discard

View File

@ -11,6 +11,8 @@ import ../raft/raft_api
export raft_api
proc RaftTimerCreateCustomImpl*(timerInterval: int, oneshot: bool, timerCallback: RaftTimerCallback): Future[void] {.async, nimcall, gcsafe.} =
await sleepAsync(timerInterval)
timerCallback()
proc RaftTimerCreateCustomImpl*(timerInterval: int, timerCallback: RaftTimerCallback): Future[void] {.async, nimcall, gcsafe.} =
var f = sleepAsync(timerInterval)
await f
if not f.cancelled:
timerCallback()

View File

@ -33,7 +33,7 @@ proc basicClusterMain*() =
var
dur: times.Duration
dur = initDuration(seconds = 5, milliseconds = 100)
waitFor sleepAsync(5000)
waitFor sleepAsync(500)
test "Simulate Basic Raft Cluster Client SmCommands Execution / Log Replication":
discard

View File

@ -38,16 +38,16 @@ proc basicTimersMain*() =
test "Create 'slow' and 'fast' timers":
for i in 0..MAX_TIMERS:
slowTimers[i] = RaftTimerCreateCustomImpl(max(SLOW_TIMERS_MIN, rand(SLOW_TIMERS_MAX)), true, RaftTimerCallbackCnt(slowCnt))
slowTimers[i] = RaftTimerCreateCustomImpl(max(SLOW_TIMERS_MIN, rand(SLOW_TIMERS_MAX)), RaftTimerCallbackCnt(slowCnt))
for i in 0..MAX_TIMERS:
fastTimers[i] = RaftTimerCreateCustomImpl(max(FAST_TIMERS_MIN, rand(FAST_TIMERS_MAX)), true, RaftDummyTimerCallback)
fastTimers[i] = RaftTimerCreateCustomImpl(max(FAST_TIMERS_MIN, rand(FAST_TIMERS_MAX)), RaftDummyTimerCallback)
test "Wait for and cancel 'slow' timers":
waitFor sleepAsync(WAIT_FOR_SLOW_TIMERS)
for i in 0..MAX_TIMERS:
if not slowTimers[i].finished:
cancel(slowTimers[i])
asyncSpawn cancelAndWait(slowTimers[i])
test "Final wait timers":
waitFor sleepAsync(FINAL_WAIT)