Add basic Raft Cluster test

This commit is contained in:
Raycho Mukelov 2023-08-31 23:52:52 +03:00
parent 79e96d40a5
commit 634094f766
9 changed files with 123 additions and 41 deletions

3
.gitignore vendored
View File

@ -5,4 +5,5 @@
*.exe *.exe
*.out *.out
nimcache/ nimcache/
build/ build/
nimbledeps/

View File

@ -1,9 +1,11 @@
# Nimbus # nim-raft
# Copyright (c) 2018-2019 Status Research & Development GmbH # Copyright (c) 2023 Status Research & Development GmbH
# Licensed under either of # Licensed under either of
# * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE) or http://www.apache.org/licenses/LICENSE-2.0) # * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE))
# * MIT license ([LICENSE-MIT](LICENSE-MIT) or http://opensource.org/licenses/MIT) # * MIT license ([LICENSE-MIT](LICENSE-MIT))
# at your option. This file may not be copied, modified, or distributed except according to those terms. # at your option.
# This file may not be copied, modified, or distributed except according to
# those terms.
import std/times import std/times
import vm_compile_info import vm_compile_info

View File

@ -1,12 +1,11 @@
# Nimbus # nim-raft
# Copyright (c) 2018 Status Research & Development GmbH # Copyright (c) 2023 Status Research & Development GmbH
# Licensed under either of # Licensed under either of
# * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE) or # * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE))
# http://www.apache.org/licenses/LICENSE-2.0) # * MIT license ([LICENSE-MIT](LICENSE-MIT))
# * MIT license ([LICENSE-MIT](LICENSE-MIT) or # at your option.
# http://opensource.org/licenses/MIT) # This file may not be copied, modified, or distributed except according to
# at your option. This file may not be copied, modified, or distributed except # those terms.
# according to those terms.
func vmName(): string = func vmName(): string =
when defined(evmc_enabled): when defined(evmc_enabled):

View File

@ -23,12 +23,6 @@ type
rmreSuccess = 0, rmreSuccess = 0,
rmreFail = 1 rmreFail = 1
RaftMessageResponseBase* = ref object of RootObj
msgId*: RaftMessageId # Original Message ID
senderId*: RaftNodeId # Sender Raft Node ID
respondentId: RaftNodeId # Responding RaftNodeId
senderTerm*: RaftNodeTerm # Sender Raft Node Term
RaftMessageRequestVote* = ref object of RaftMessageBase RaftMessageRequestVote* = ref object of RaftMessageBase
lastLogTerm*: RaftNodeTerm lastLogTerm*: RaftNodeTerm
lastLogIndex*: RaftLogIndex lastLogIndex*: RaftLogIndex
@ -59,6 +53,7 @@ type
RaftNodeClientRequest*[SmCommandType] = ref object RaftNodeClientRequest*[SmCommandType] = ref object
op*: RaftNodeClientRequestOps op*: RaftNodeClientRequestOps
nodeId*: RaftNodeId
payload*: Option[SmCommandType] # Optional RaftMessagePayload carrying a Log Entry payload*: Option[SmCommandType] # Optional RaftMessagePayload carrying a Log Entry
RaftNodeClientResponse*[SmStateType] = ref object RaftNodeClientResponse*[SmStateType] = ref object

View File

@ -17,16 +17,23 @@ proc RaftNodeSmInit[SmCommandType, SmStateType](stateMachine: var RaftNodeStateM
# Raft Node Public API procedures / functions # Raft Node Public API procedures / functions
proc RaftNodeCreateNew*[SmCommandType, SmStateType]( # Create New Raft Node proc RaftNodeCreateNew*[SmCommandType, SmStateType]( # Create New Raft Node
id: RaftNodeId, peers: RaftNodePeers, id: RaftNodeId, peersIds: seq[RaftNodeId],
persistentStorage: RaftNodePersistentStorage, # persistentStorage: RaftNodePersistentStorage,
msgSendCallback: RaftMessageSendCallback): RaftNode[SmCommandType, SmStateType] = msgSendCallback: RaftMessageSendCallback): RaftNode[SmCommandType, SmStateType] =
var var
sm = RaftNodeStateMachine[SmCommandType, SmStateType] sm: RaftNodeStateMachine[SmCommandType, SmStateType]
peers: RaftNodePeers
RaftNodeSmInit[SmCommandType, SmStateType](sm) RaftNodeSmInit[SmCommandType, SmStateType](sm)
for peerId in peersIds:
peers.add(RaftNodePeer(id: peerId, nextIndex: 0, matchIndex: 0, hasVoted: false, canVote: true))
result = RaftNode[SmCommandType, SmStateType]( result = RaftNode[SmCommandType, SmStateType](
id: id, state: rnsFollower, currentTerm: 0, votedFor: nil, peers: peers, commitIndex: 0, lastApplied: 0, id: id, state: rnsFollower, currentTerm: 0, peers: peers, commitIndex: 0, lastApplied: 0,
stateMachine: sm, msgSendCallback: msgSendCallback stateMachine: sm, msgSendCallback: msgSendCallback
) )
initLock(result.raftStateMutex)
proc RaftNodeLoad*[SmCommandType, SmStateType]( proc RaftNodeLoad*[SmCommandType, SmStateType](
persistentStorage: RaftNodePersistentStorage, # Load Raft Node From Storage persistentStorage: RaftNodePersistentStorage, # Load Raft Node From Storage

View File

@ -14,7 +14,7 @@ import options
import stew/results import stew/results
import uuids import uuids
export results, options export results, options, locks, uuids
type type
@ -24,7 +24,6 @@ type
rnsCandidate = 2 rnsCandidate = 2
rnsLeader = 3 rnsLeader = 3
UUID = object
RaftNodeId* = UUID # uuid4 uniquely identifying every Raft Node RaftNodeId* = UUID # uuid4 uniquely identifying every Raft Node
RaftNodeTerm* = uint64 # Raft Node Term Type RaftNodeTerm* = uint64 # Raft Node Term Type
RaftLogIndex* = uint64 # Raft Node Log Index Type RaftLogIndex* = uint64 # Raft Node Log Index Type
@ -71,10 +70,17 @@ type
RaftMessageBase* = ref object of RootObj # Base Type for Raft Protocol Messages RaftMessageBase* = ref object of RootObj # Base Type for Raft Protocol Messages
msgId*: RaftMessageId # Message UUID msgId*: RaftMessageId # Message UUID
senderId*: RaftNodeId # Sender Raft Node ID senderId*: RaftNodeId # Sender Raft Node ID
receiverId*: RaftNodeId # Receiver Raft Node ID
senderTerm*: RaftNodeTerm # Sender Raft Node Term senderTerm*: RaftNodeTerm # Sender Raft Node Term
RaftMessageSendCallback* = proc (recipient: RaftNodeId, raftMessage: RaftMessageBase) {.nimcall, gcsafe.} # Callback for Sending Raft Node Messages RaftMessageResponseBase* = ref object of RootObj
# out of this Raft Node. msgId*: RaftMessageId # Original Message ID
senderId*: RaftNodeId # Sender Raft Node ID
respondentId: RaftNodeId # Responding RaftNodeId
senderTerm*: RaftNodeTerm # Sender Raft Node Term
RaftMessageSendCallback* = proc (raftMessage: RaftMessageBase): RaftMessageResponseBase {.gcsafe.} # Callback for Sending Raft Node Messages
# out of this Raft Node.
# For later use when adding/removing new nodes (dynamic configuration chganges) # For later use when adding/removing new nodes (dynamic configuration chganges)
RaftNodeConfiguration* = object RaftNodeConfiguration* = object
@ -120,7 +126,7 @@ type
appendEntriesTimer: RaftTimer appendEntriesTimer: RaftTimer
# Mtx definition(s) go here # Mtx definition(s) go here
raftStateMutex: Lock raftStateMutex*: Lock
# Modules (Algos) # Modules (Algos)
consensusModule: RaftConsensusModule[SmCommandType, SmStateType] consensusModule: RaftConsensusModule[SmCommandType, SmStateType]
@ -128,22 +134,22 @@ type
membershipChangeModule: RaftMembershipChangeModule[SmCommandType, SmStateType] membershipChangeModule: RaftMembershipChangeModule[SmCommandType, SmStateType]
# Misc # Misc
msgSendCallback: RaftMessageSendCallback msgSendCallback*: RaftMessageSendCallback
persistentStorage: RaftNodePersistentStorage[SmCommandType, SmStateType] persistentStorage: RaftNodePersistentStorage[SmCommandType, SmStateType]
# Persistent state # Persistent state
id: RaftNodeId # This Raft Node ID id*: RaftNodeId # This Raft Node ID
state: RaftNodeState # This Raft Node State state*: RaftNodeState # This Raft Node State
currentTerm: RaftNodeTerm # Latest term this Raft Node has seen (initialized to 0 on first boot, increases monotonically) currentTerm*: RaftNodeTerm # Latest term this Raft Node has seen (initialized to 0 on first boot, increases monotonically)
votedFor: RaftNodeId # Candidate RaftNodeId that received vote in current term (or nil/zero if none), votedFor*: RaftNodeId # Candidate RaftNodeId that received vote in current term (or nil/zero if none),
# also used to redirect Client Requests in case this Raft Node is not the leader # also used to redirect Client Requests in case this Raft Node is not the leader
log: RaftNodeLog[SmCommandType] # This Raft Node Log log: RaftNodeLog[SmCommandType] # This Raft Node Log
stateMachine: RaftNodeStateMachine[SmCommandType, SmStateType] # Not sure for now putting it here. I assume that persisting the State Machine's stateMachine*: RaftNodeStateMachine[SmCommandType, SmStateType] # Not sure for now putting it here. I assume that persisting the State Machine's
# state is enough to consider it 'persisted' # state is enough to consider it 'persisted'
peers: RaftNodePeers # This Raft Node Peers IDs. I am not sure if this must be persistent or volatile but making it persistent peers*: RaftNodePeers # This Raft Node Peers IDs. I am not sure if this must be persistent or volatile but making it persistent
# makes sense for the moment # makes sense for the moment
# Volatile state # Volatile state
commitIndex: RaftLogIndex # Index of highest log entry known to be committed (initialized to 0, increases monotonically) commitIndex*: RaftLogIndex # Index of highest log entry known to be committed (initialized to 0, increases monotonically)
lastApplied: RaftLogIndex # Index of highest log entry applied to state machine (initialized to 0, increases monotonically) lastApplied*: RaftLogIndex # Index of highest log entry applied to state machine (initialized to 0, increases monotonically)
currentLeaderId: RaftNodeId # The ID of the cirrent leader Raft Node or 0/nil if None is leader (election is in progress etc.) currentLeaderId: RaftNodeId # The ID of the cirrent leader Raft Node or 0/nil if None is leader (election is in progress etc.)

View File

@ -7,10 +7,11 @@
# This file may not be copied, modified, or distributed except according to # This file may not be copied, modified, or distributed except according to
# those terms. # those terms.
import test_macro import ../misc/test_macro
{. warning[UnusedImport]: off .} {. warning[UnusedImport]:off .}
cliBuilder: cliBuilder:
import ./test_basic_state_machine, import ./test_basic_cluster,
./test_basic_state_machine,
./test_basic_timers ./test_basic_timers

43
tests/basic_cluster.nim Normal file
View File

@ -0,0 +1,43 @@
# nim-raft
# Copyright (c) 2023 Status Research & Development GmbH
# Licensed under either of
# * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE))
# * MIT license ([LICENSE-MIT](LICENSE-MIT))
# at your option.
# This file may not be copied, modified, or distributed except according to
# those terms.
import basic_timers
import basic_state_machine
import ../raft/raft_api
type
BasicRaftNode* = RaftNode[SmCommand, SmState]
BasicRaftCluster* = ref object
nodes*: seq[BasicRaftNode]
proc BasicRaftClusterRaftMessageSendCallbackCreate(cluster: BasicRaftCluster): RaftMessageSendCallback =
proc (msg: RaftMessageBase): RaftMessageResponseBase {.closure.} =
var
nodeIdx: int = -1
for i in 0..cluster.nodes.len:
if cluster.nodes[i].id == msg.receiverId:
nodeIdx = i
break
cluster.nodes[nodeIdx].RaftNodeMessageDeliver(msg)
proc BasicRaftClusterClientRequest*(cluster: BasicRaftCluster, req: RaftNodeClientRequest): RaftNodeClientResponse =
discard
proc BasicRaftClusterInit*(nodesIds: seq[RaftNodeId]): BasicRaftCluster =
new(result)
for nodeId in nodesIds:
var
peersIds = nodesIds
peersIds.del(peersIds.find(nodeId))
result.nodes.add(RaftNodeCreateNew[SmCommand, SmState](nodeId, peersIds, BasicRaftClusterRaftMessageSendCallbackCreate(result)))

View File

@ -0,0 +1,28 @@
# nim-raft
# Copyright (c) 2023 Status Research & Development GmbH
# Licensed under either of
# * Apache License, version 2.0, ([LICENSE-APACHE](LICENSE-APACHE))
# * MIT license ([LICENSE-MIT](LICENSE-MIT))
# at your option.
# This file may not be copied, modified, or distributed except according to
# those terms.
import unittest2
import basic_cluster
import ../raft/types
proc basicClusterMain*() =
var
cluster: BasicRaftCluster
nodesIds = newSeq[RaftNodeId](5)
suite "Basic Raft Cluster Tests":
test "Test Basic Raft Cluster Init (5 nodes)":
for i in 0..4:
nodesIds[i] = genUUID()
cluster = BasicRaftClusterInit(nodesIds)
if isMainModule:
basicClusterMain()