Implement appendentries fully
This commit is contained in:
@@ -19,7 +19,10 @@ type Cluster<'a> =
|
||||
this.Servers.[i / 1<ServerId>].TriggerHeartbeatTimeout ()
|
||||
this.Servers.[i / 1<ServerId>].Sync ()
|
||||
|
||||
member this.State (i : int<ServerId>) : ServerStatus = this.Servers.[i / 1<ServerId>].State
|
||||
member this.Status (i : int<ServerId>) : ServerStatus = this.Servers.[i / 1<ServerId>].State
|
||||
|
||||
member this.GetCurrentInternalState (i : int<ServerId>) : ServerInternalState<'a> Async =
|
||||
this.Servers.[i / 1<ServerId>].GetCurrentInternalState ()
|
||||
|
||||
member this.ClusterSize : int = this.Servers.Length
|
||||
|
||||
|
137
Raft/Server.fs
137
Raft/Server.fs
@@ -20,18 +20,34 @@ type VolatileState =
|
||||
|
||||
type LeaderState =
|
||||
{
|
||||
/// For each server, index of the next log entry to send to that server
|
||||
NextIndex : int<LogIndex> array
|
||||
/// For each server, index of the log entry to send to them next. Note that this might not
|
||||
/// actually be the *first* index we need to send - the recipient may reject this message.
|
||||
/// When they reject this message, we'll decrement ToSend and try again with an earlier
|
||||
/// message, until eventually we go far back enough in time that our log intersects with
|
||||
/// that of the recipient, and they'll accept it.
|
||||
ToSend : int<LogIndex> array
|
||||
/// For each server, index of the highest log entry known to be replicated on that server
|
||||
MatchIndex : int<LogIndex> array
|
||||
}
|
||||
|
||||
static member New (clusterSize : int) (currentIndex : int<LogIndex>) : LeaderState =
|
||||
{
|
||||
NextIndex = Array.create clusterSize (currentIndex + 1<LogIndex>)
|
||||
// +1, because these are indexed from 1.
|
||||
ToSend = Array.create clusterSize (currentIndex + 1<LogIndex>)
|
||||
MatchIndex = Array.zeroCreate clusterSize
|
||||
}
|
||||
|
||||
member this.Clone () =
|
||||
let cloneArray (arr : 'b array) : 'b array =
|
||||
let result = Array.zeroCreate<'b> arr.Length
|
||||
System.Array.Copy (arr, result, arr.Length)
|
||||
result
|
||||
|
||||
{
|
||||
ToSend = cloneArray this.ToSend
|
||||
MatchIndex = cloneArray this.MatchIndex
|
||||
}
|
||||
|
||||
/// You asked me to vote for you to become leader. Here is my response.
|
||||
type RequestVoteReply =
|
||||
{
|
||||
@@ -99,6 +115,8 @@ type AppendEntriesMessage<'a> =
|
||||
/// with what happened during terms that took place while it was down.
|
||||
NewEntry : ('a * int<Term>) option
|
||||
LeaderCommitIndex : int<LogIndex>
|
||||
/// TODO - we don't need this, the responder should just construct
|
||||
/// the appropriate Message and send it themselves
|
||||
ReplyChannel : AppendEntriesReply -> unit
|
||||
}
|
||||
|
||||
@@ -115,6 +133,17 @@ type AppendEntriesMessage<'a> =
|
||||
this.LeaderTerm
|
||||
this.LeaderCommitIndex
|
||||
|
||||
/// A readout of the server's internal state, suitable for e.g. debugging tools.
|
||||
type ServerInternalState<'a> =
|
||||
{
|
||||
LogIndex : int<LogIndex>
|
||||
CurrentTerm : int<Term>
|
||||
CurrentVote : int<ServerId> option
|
||||
Log : ('a * int<Term>) option list
|
||||
/// A clone of the leader state, if this is a leader.
|
||||
LeaderState : LeaderState option
|
||||
}
|
||||
|
||||
type ClientReply =
|
||||
/// You asked a node that isn't the leader. Here's a hint about whom you should ask instead.
|
||||
/// The hint may not be accurate even as of the time when we reply, and certainly it may not be
|
||||
@@ -208,6 +237,7 @@ type private ServerAction<'a> =
|
||||
| EmitHeartbeat
|
||||
| Receive of Message<'a>
|
||||
| Sync of AsyncReplyChannel<unit>
|
||||
| StateReadout of AsyncReplyChannel<ServerInternalState<'a>>
|
||||
|
||||
[<RequireQualifiedAccess>]
|
||||
module internal ServerUtils =
|
||||
@@ -468,30 +498,39 @@ type Server<'a>
|
||||
assert (logIsConsistent message)
|
||||
acceptRequest ()
|
||||
|
||||
let sendAppendEntries (leaderState : LeaderState) (j : int<ServerId>) =
|
||||
let toSend = leaderState.ToSend.[j / 1<ServerId>]
|
||||
let prevLogTerm = persistentState.GetLogEntry (toSend - 1<LogIndex>)
|
||||
|
||||
{
|
||||
LeaderTerm = persistentState.CurrentTerm
|
||||
LeaderId = me
|
||||
PrevLogEntry =
|
||||
match prevLogTerm with
|
||||
| None -> None
|
||||
| Some (_, term) ->
|
||||
{
|
||||
Term = term
|
||||
Index = toSend - 1<LogIndex>
|
||||
}
|
||||
|> Some
|
||||
NewEntry = persistentState.GetLogEntry toSend
|
||||
LeaderCommitIndex = volatileState.CommitIndex
|
||||
ReplyChannel = fun reply -> reply |> Reply.AppendEntriesReply |> Message.Reply |> messageChannel me
|
||||
}
|
||||
|> Instruction.AppendEntries
|
||||
|> Message.Instruction
|
||||
|> messageChannel j
|
||||
|
||||
let divideByTwoRoundingUp (n : int) =
|
||||
if n % 2 = 0 then n / 2 else (n / 2) + 1
|
||||
|
||||
let emitHeartbeat () =
|
||||
match currentType with
|
||||
| ServerSpecialisation.Candidate _
|
||||
| ServerSpecialisation.Follower _ -> ()
|
||||
| ServerSpecialisation.Leader _ ->
|
||||
let lastLogEntry = persistentState.GetLastLogEntry () |> Option.map snd
|
||||
let emitHeartbeat (leaderState : LeaderState) =
|
||||
for i in 0 .. clusterSize - 1 do
|
||||
let i = i * 1<ServerId>
|
||||
|
||||
for i in 0 .. clusterSize - 1 do
|
||||
if i * 1<ServerId> <> me then
|
||||
{
|
||||
LeaderTerm = persistentState.CurrentTerm
|
||||
LeaderId = me
|
||||
PrevLogEntry = lastLogEntry
|
||||
NewEntry = None
|
||||
LeaderCommitIndex = volatileState.CommitIndex
|
||||
ReplyChannel =
|
||||
fun reply -> messageChannel me (reply |> Reply.AppendEntriesReply |> Message.Reply)
|
||||
}
|
||||
|> Instruction.AppendEntries
|
||||
|> Message.Instruction
|
||||
|> messageChannel (i * 1<ServerId>)
|
||||
if i <> me then
|
||||
sendAppendEntries leaderState i
|
||||
|
||||
let processReply (r : Reply) : unit =
|
||||
match r with
|
||||
@@ -504,13 +543,15 @@ type Server<'a>
|
||||
if appendEntriesReply.FollowerTerm = persistentState.CurrentTerm then
|
||||
match appendEntriesReply.Success with
|
||||
| Some matchIndex ->
|
||||
// They applied our request. Update our record of what we know they have applied...
|
||||
leaderState.MatchIndex.[appendEntriesReply.Follower / 1<ServerId>] <- matchIndex
|
||||
leaderState.NextIndex.[appendEntriesReply.Follower / 1<ServerId>] <- matchIndex + 1<LogIndex>
|
||||
// ... and update our record of what we'll be sending them next.
|
||||
leaderState.ToSend.[appendEntriesReply.Follower / 1<ServerId>] <- matchIndex + 1<LogIndex>
|
||||
| None ->
|
||||
leaderState.NextIndex.[appendEntriesReply.Follower / 1<ServerId>] <-
|
||||
max
|
||||
(leaderState.NextIndex.[appendEntriesReply.Follower / 1<ServerId>] - 1<LogIndex>)
|
||||
1<LogIndex>
|
||||
// They failed to apply our request. Next time, we'll be trying one message further
|
||||
// back in our history.
|
||||
leaderState.ToSend.[appendEntriesReply.Follower / 1<ServerId>] <-
|
||||
max (leaderState.ToSend.[appendEntriesReply.Follower / 1<ServerId>] - 1<LogIndex>) 1<LogIndex>
|
||||
// Note that the decision to process this *here* means the algorithm doesn't work in clusters of
|
||||
// only one node, because then there will never be any AppendEntries replies.
|
||||
let maxLogAQuorumHasCommitted =
|
||||
@@ -551,11 +592,10 @@ type Server<'a>
|
||||
state.Votes > clusterSize / 2
|
||||
then
|
||||
// Become the leader!
|
||||
currentType <-
|
||||
LeaderState.New clusterSize persistentState.CurrentLogIndex
|
||||
|> ServerSpecialisation.Leader
|
||||
let state = LeaderState.New clusterSize persistentState.CurrentLogIndex
|
||||
currentType <- ServerSpecialisation.Leader state
|
||||
|
||||
emitHeartbeat ()
|
||||
emitHeartbeat state
|
||||
|
||||
let mailbox =
|
||||
let rec loop (mailbox : MailboxProcessor<_>) =
|
||||
@@ -565,7 +605,12 @@ type Server<'a>
|
||||
//System.Console.WriteLine toPrint
|
||||
|
||||
match m with
|
||||
| ServerAction.EmitHeartbeat -> emitHeartbeat ()
|
||||
| ServerAction.EmitHeartbeat ->
|
||||
match currentType with
|
||||
| ServerSpecialisation.Leader state -> emitHeartbeat state
|
||||
| ServerSpecialisation.Candidate _
|
||||
| ServerSpecialisation.Follower _ -> ()
|
||||
|
||||
| ServerAction.BeginElection ->
|
||||
match currentType with
|
||||
| ServerSpecialisation.Leader _ -> ()
|
||||
@@ -592,13 +637,32 @@ type Server<'a>
|
||||
| ServerAction.Receive (Message.Reply r) -> processReply r
|
||||
| ServerAction.Receive (Message.ClientRequest (toAdd, replyChannel)) ->
|
||||
match currentType with
|
||||
| ServerSpecialisation.Leader _ ->
|
||||
| ServerSpecialisation.Leader leaderState ->
|
||||
persistentState.AppendToLog toAdd persistentState.CurrentTerm
|
||||
replyChannel ClientReply.Acknowledged
|
||||
emitHeartbeat leaderState
|
||||
| ServerSpecialisation.Follower followerState ->
|
||||
replyChannel (ClientReply.Redirect followerState.CurrentLeader)
|
||||
| ServerSpecialisation.Candidate _ -> replyChannel ClientReply.Dropped
|
||||
| ServerAction.Sync replyChannel -> replyChannel.Reply ()
|
||||
| ServerAction.StateReadout replyChannel ->
|
||||
{
|
||||
LogIndex = persistentState.CurrentLogIndex
|
||||
CurrentTerm = persistentState.CurrentTerm
|
||||
CurrentVote = persistentState.VotedFor
|
||||
Log =
|
||||
match persistentState.GetLastLogEntry () with
|
||||
| None -> []
|
||||
| Some (_, last) ->
|
||||
List.init
|
||||
(last.Index / 1<LogIndex>)
|
||||
(fun index -> persistentState.GetLogEntry (1<LogIndex> + index * 1<LogIndex>))
|
||||
LeaderState =
|
||||
match currentType with
|
||||
| ServerSpecialisation.Leader state -> state.Clone () |> Some
|
||||
| _ -> None
|
||||
}
|
||||
|> replyChannel.Reply
|
||||
|
||||
return! loop mailbox
|
||||
}
|
||||
@@ -617,6 +681,11 @@ type Server<'a>
|
||||
|
||||
member this.Message (m : Message<'a>) = mailbox.Post (ServerAction.Receive m)
|
||||
|
||||
member this.GetCurrentInternalState () : Async<ServerInternalState<'a>> =
|
||||
mailbox.PostAndAsyncReply ServerAction.StateReadout
|
||||
|
||||
member this.PersistentState = persistentState
|
||||
|
||||
member this.Sync () =
|
||||
// This rather eccentric phrasing is so that Fable can run this mailbox.
|
||||
// (Fable does not support `mailbox.PostAndReply`, nor does it support
|
||||
|
@@ -18,7 +18,7 @@ module Program =
|
||||
|
||||
let printClusterState<'a> (cluster : Cluster<'a>) : unit =
|
||||
for i in 0 .. cluster.ClusterSize - 1 do
|
||||
printfn "Server %i: %O" i (cluster.State (i * 1<ServerId>))
|
||||
printfn "Server %i: %O" i (cluster.Status (i * 1<ServerId>))
|
||||
|
||||
let getMessage (clusterSize : int) (s : string) : (int<ServerId> * int) option =
|
||||
match s.Split ',' with
|
||||
@@ -200,7 +200,7 @@ module Program =
|
||||
clusterSize
|
||||
(fun i ->
|
||||
let i = i * 1<ServerId>
|
||||
i, cluster.State i
|
||||
i, cluster.Status i
|
||||
)
|
||||
|> Seq.choose (fun (i, status) ->
|
||||
match status with
|
||||
|
@@ -6,9 +6,24 @@
|
||||
<meta name="viewport" content="width=device-width, initial-scale=1">
|
||||
</head>
|
||||
<body>
|
||||
<p class="startup-text">Starting up...</p>
|
||||
<h2>Status of each server</h2>
|
||||
<table class="server-statuses">
|
||||
</table>
|
||||
<p class="startup-text">Fable is running</p>
|
||||
<h2>Logs stored on each server</h2>
|
||||
<table class="log-area">
|
||||
</table>
|
||||
<h2>Leader information</h2>
|
||||
<div class="leader-area">
|
||||
Leader information for server <b class="leader-state"></b>
|
||||
<form>
|
||||
<input type="number" class="leader-select" />
|
||||
<button id="leader-select-button" class="leader-select-button" type="button">Display info for this leader</button>
|
||||
</form>
|
||||
<table class="leader-state-table">
|
||||
</table>
|
||||
</div>
|
||||
<h2>Interaction</h2>
|
||||
<form>
|
||||
<input type="number" class="timeout-text" />
|
||||
<button id="timeout-button" class="timeout-button" type="button">Inactivity timeout server</button>
|
||||
@@ -24,6 +39,7 @@
|
||||
<input type="number" id="client-data" class="client-data" />
|
||||
<button class="client-data-submit" type="button">Submit client data</button>
|
||||
</form>
|
||||
<h2>Messages in flight</h2>
|
||||
<table class="button-area"></table>
|
||||
<script src="bundle.js"></script>
|
||||
</body>
|
||||
|
@@ -1,9 +1,9 @@
|
||||
namespace RaftFable
|
||||
|
||||
|
||||
open Fable.Core.JS
|
||||
open Raft
|
||||
open Browser.Dom
|
||||
open Fable.Core
|
||||
|
||||
module App =
|
||||
|
||||
@@ -27,6 +27,42 @@ module App =
|
||||
statusCell
|
||||
|> List.init clusterSize
|
||||
|
||||
let logArea = document.querySelector ".log-area" :?> Browser.Types.HTMLTableElement
|
||||
|
||||
logArea.border <- "1px"
|
||||
|
||||
let renderLogArea (cluster : Cluster<'a>) =
|
||||
logArea.innerText <- ""
|
||||
|
||||
let rows =
|
||||
List.init
|
||||
cluster.ClusterSize
|
||||
(fun i ->
|
||||
let cell = document.createElement "th" :?> Browser.Types.HTMLTableCellElement
|
||||
cell.textContent <- sprintf "Server %i" i
|
||||
Table.createRow document [ Some cell ] logArea
|
||||
)
|
||||
|
||||
rows
|
||||
|> List.iteri (fun i row ->
|
||||
let i = i * 1<ServerId>
|
||||
|
||||
cluster.GetCurrentInternalState i
|
||||
|> Async.StartAsPromise
|
||||
|> fun p ->
|
||||
p.``then`` (fun state ->
|
||||
for logEntry in state.Log do
|
||||
let cell = document.createElement "td" :?> Browser.Types.HTMLTableCellElement
|
||||
|
||||
match logEntry with
|
||||
| None -> cell.textContent <- "<none>"
|
||||
| Some (value, term) -> cell.textContent <- sprintf "%i: %O" term value
|
||||
|
||||
row.appendChild cell |> ignore
|
||||
)
|
||||
|> ignore
|
||||
)
|
||||
|
||||
let messageQueueArea =
|
||||
document.querySelector ".button-area" :?> Browser.Types.HTMLTableElement
|
||||
|
||||
@@ -40,15 +76,86 @@ module App =
|
||||
|
||||
resetButtonArea ()
|
||||
|
||||
let setLeaderState (cluster : Cluster<'a>) (id : int<ServerId>) =
|
||||
let leaderIdBox =
|
||||
document.querySelector ".leader-state" :?> Browser.Types.HTMLBlockElement
|
||||
|
||||
leaderIdBox.innerText <- sprintf "%i" id
|
||||
|
||||
let leaderIdTable =
|
||||
document.querySelector ".leader-state-table" :?> Browser.Types.HTMLTableElement
|
||||
|
||||
leaderIdTable.innerText <- ""
|
||||
|
||||
leaderIdTable
|
||||
|> Table.createHeaderRow document ("" :: List.init clusterSize (sprintf "Server %i"))
|
||||
|
||||
let leaderStatePromise = cluster.GetCurrentInternalState id |> Async.StartAsPromise
|
||||
|
||||
leaderStatePromise.``then`` (fun state ->
|
||||
match state.LeaderState with
|
||||
| None -> leaderIdBox.innerText <- sprintf "%i: not a leader" id
|
||||
| Some state ->
|
||||
|
||||
let knownStoredIndices =
|
||||
state.MatchIndex
|
||||
|> Seq.mapi (fun target index ->
|
||||
let target = target * 1<ServerId>
|
||||
if target = id then "(self)" else sprintf "%i" index
|
||||
)
|
||||
|> Seq.toList
|
||||
|> fun l -> "Log index known to be stored on each node" :: l
|
||||
|> List.map (fun text ->
|
||||
let node = document.createElement "div"
|
||||
node.innerText <- text
|
||||
Some node
|
||||
)
|
||||
|
||||
Table.createRow document knownStoredIndices leaderIdTable |> ignore
|
||||
|
||||
let nextToSend =
|
||||
state.ToSend
|
||||
|> Seq.mapi (fun target index ->
|
||||
let target = target * 1<ServerId>
|
||||
if target = id then "(self)" else sprintf "%i" index
|
||||
)
|
||||
|> Seq.toList
|
||||
|> fun l -> "Will try next to send this index" :: l
|
||||
|> List.map (fun text ->
|
||||
let node = document.createElement "div"
|
||||
node.innerText <- text
|
||||
Some node
|
||||
)
|
||||
|
||||
Table.createRow document nextToSend leaderIdTable |> ignore
|
||||
)
|
||||
|
||||
let printClusterState<'a> (cluster : Cluster<'a>) : unit =
|
||||
for i in 0 .. cluster.ClusterSize - 1 do
|
||||
serverStatusNodes.[i].textContent <- cluster.State (i * 1<ServerId>) |> string<ServerStatus>
|
||||
let status = cluster.Status (i * 1<ServerId>)
|
||||
serverStatusNodes.[i].textContent <- status |> string<ServerStatus>
|
||||
|
||||
let cluster, network = InMemoryCluster.make<byte> clusterSize
|
||||
|
||||
let leaderStateButton =
|
||||
document.querySelector ".leader-select-button" :?> Browser.Types.HTMLButtonElement
|
||||
|
||||
let selectedLeaderId =
|
||||
document.querySelector ".leader-select" :?> Browser.Types.HTMLInputElement
|
||||
|
||||
selectedLeaderId.min <- "0"
|
||||
selectedLeaderId.max <- sprintf "%i" (clusterSize - 1)
|
||||
selectedLeaderId.defaultValue <- "0"
|
||||
|
||||
leaderStateButton.onclick <-
|
||||
fun _ ->
|
||||
let id = selectedLeaderId.valueAsNumber |> int |> (fun i -> i * 1<ServerId>)
|
||||
setLeaderState cluster id
|
||||
|
||||
let performWithoutPrintingNetworkState action =
|
||||
NetworkAction.perform cluster network action
|
||||
printClusterState cluster
|
||||
renderLogArea cluster
|
||||
|
||||
let rec printNetworkState<'a> (network : Network<'a>) : unit =
|
||||
resetButtonArea ()
|
||||
@@ -79,7 +186,7 @@ module App =
|
||||
|> List.transpose
|
||||
|
||||
for row in allButtons' do
|
||||
Table.createRow document row messageQueueArea
|
||||
Table.createRow document row messageQueueArea |> ignore
|
||||
|
||||
let perform (action : NetworkAction<_>) : unit =
|
||||
performWithoutPrintingNetworkState action
|
||||
@@ -88,8 +195,6 @@ module App =
|
||||
let startupText =
|
||||
document.querySelector ".startup-text" :?> Browser.Types.HTMLParagraphElement
|
||||
|
||||
startupText.textContent <- "Starting up..."
|
||||
|
||||
let startupSequence =
|
||||
[
|
||||
NetworkAction.InactivityTimeout 0<ServerId>
|
||||
@@ -123,7 +228,7 @@ module App =
|
||||
]
|
||||
|> fun s -> (Constructors.Promise.resolve (printClusterState cluster), s)
|
||||
||> List.fold (fun (inPromise : Promise<unit>) action -> inPromise.``then`` (fun () -> perform action))
|
||||
|> fun p -> p.``then`` (fun () -> startupText.textContent <- "Started! Press buttons.")
|
||||
|> fun p -> p.``then`` (fun () -> startupText.textContent <- "")
|
||||
|
||||
let timeoutButton =
|
||||
document.querySelector ".timeout-button" :?> Browser.Types.HTMLButtonElement
|
||||
@@ -144,6 +249,8 @@ module App =
|
||||
|> InactivityTimeout
|
||||
|> perform
|
||||
|
||||
printClusterState cluster
|
||||
renderLogArea cluster
|
||||
printNetworkState network
|
||||
)
|
||||
|
||||
@@ -166,6 +273,8 @@ module App =
|
||||
|> Heartbeat
|
||||
|> perform
|
||||
|
||||
printClusterState cluster
|
||||
renderLogArea cluster
|
||||
printNetworkState network
|
||||
)
|
||||
|
||||
@@ -196,4 +305,6 @@ module App =
|
||||
NetworkAction.ClientRequest (server, data, printfn "%O") |> perform
|
||||
|
||||
printClusterState cluster
|
||||
renderLogArea cluster
|
||||
printNetworkState network
|
||||
)
|
||||
|
@@ -22,7 +22,7 @@ module Table =
|
||||
(document : Browser.Types.Document)
|
||||
(elements : seq<#Browser.Types.Node option>)
|
||||
(table : Browser.Types.HTMLTableElement)
|
||||
: unit =
|
||||
: Browser.Types.HTMLTableRowElement =
|
||||
let row = document.createElement "tr" :?> Browser.Types.HTMLTableRowElement
|
||||
|
||||
for col in elements do
|
||||
@@ -35,3 +35,4 @@ module Table =
|
||||
row.appendChild entry |> ignore
|
||||
|
||||
table.appendChild row |> ignore
|
||||
row
|
||||
|
Reference in New Issue
Block a user