raftkvs

package module
v0.0.0-...-2e7796a Latest Latest
Warning

This package is not in the latest version of its module.

Go to latest
Published: Apr 6, 2023 License: Apache-2.0 Imports: 11 Imported by: 0

README

raftkvs

raftkvs is a key-value store based on the Raft consensus protocol.

Model

raftkvs has two set of nodes: server and client.

A server has several archetypes that run concurrently. AServerHandler handles the incoming messages in a server. AServerRequestVote starts a new election round in case of a leader timeout. AServerBecomeLeader detects if the current server has a quorum of votes and then promotes itself to be the leader. If the current server is the leader, AServerAppendEntries broadcasts new entries, and AServerAdvanceCommitIndex increases the commit index.

A client has an archetype AClient that receives input requests from a channel (reqCh) and sends them to a server. Then it receives the response and returns it to the user via another channel (respCh).

Assumptions

The makes the following assumption for the environment during model checking:

  1. Network links are reliable FIFO links, which is modeled by ReliableFIFOLink mapping macro.
  2. The failure detector used is a theoretically perfect failure detector.
  3. Servers might fail with respect to the crash-stop failure semantics. Crashed servers cannot rejoin the system.
  4. Reconfiguration is not supported, therefore, the set of servers is static.
Properties

The following five properties are the properties defined by the original Raft paper.

  • ElectionSafety: at most one leader can be elected in a given term.
  • LogMatching: if two logs contain an entry with the same index and term, then the logs are identical in all entries up through the given index.
  • LeaderCompleteness: if a log entry is committed in a given term, then that entry will be present in the logs of the leaders for all higher-numbered terms.
  • StateMachineSafety: if a server has applied a log entry at a given index to its state machine, no other server will ever apply a different log entry for the same index.
  • LeaderAppendOnly: a leader never overwrites or deletes entries in its log; it only appends new entries.

Moreover, we have defined some extra properties as well:

  • ApplyLogOK: committed log entries are correctly reflected in the key-value store dictionary.
  • plogOK: persistent log is update correctly.
  • ClientsOK: clients will eventually receive a response for the corresponding request. This property will be violated if servers can crash, i.e., ExploreFail set to be true.
  • ElectionLiveness: eventually a server will be elected as the leader. This will be violated if the number of servers are more than one.
Constraints

LimitTerm and LimitCommitIndex limit servers term and commit index respectively. LimitNodeFailure limits the number of crashing nodes.

Usage

Verification

Configurations can be found in the models folder.

make mc  # TLC model checking
make sim # TLC simulation, useful for fiding 
Build
make build
Run

Sample config files can be found in the configs folder.

./server -srvId <server_id> -c <path/to/config.yaml>
./client -clientId <client_id> -c <path/to/config.yaml>
Test
make test
make racetest
Benchmark

Our fork of Go YCSB supports raftkvs. For more details, check it out.

Documentation

Index

Constants

This section is empty.

Variables

View Source
var AClient = distsys.MPCalArchetype{
	Name:              "AClient",
	Label:             "AClient.clientLoop",
	RequiredRefParams: []string{"AClient.net", "AClient.netLen", "AClient.fd", "AClient.reqCh", "AClient.respCh", "AClient.timeout"},
	RequiredValParams: []string{},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
		iface.EnsureArchetypeResourceLocal("AClient.leader", Nil(iface))
		iface.EnsureArchetypeResourceLocal("AClient.req", tla.Value{})
		iface.EnsureArchetypeResourceLocal("AClient.resp", tla.Value{})
		iface.EnsureArchetypeResourceLocal("AClient.reqIdx", tla.MakeNumber(0))
	},
}
View Source
var AServer = distsys.MPCalArchetype{
	Name:              "AServer",
	Label:             "AServer.serverLoop",
	RequiredRefParams: []string{"AServer.net", "AServer.netLen", "AServer.netEnabled", "AServer.fd", "AServer.state", "AServer.currentTerm", "AServer.log", "AServer.plog", "AServer.commitIndex", "AServer.nextIndex", "AServer.matchIndex", "AServer.votedFor", "AServer.votesResponded", "AServer.votesGranted", "AServer.leader", "AServer.sm", "AServer.smDomain", "AServer.leaderTimeout", "AServer.appendEntriesCh", "AServer.becomeLeaderCh"},
	RequiredValParams: []string{"AServer.srvId"},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
		iface.EnsureArchetypeResourceLocal("AServer.idx", tla.MakeNumber(1))
		iface.EnsureArchetypeResourceLocal("AServer.m", tla.Value{})
	},
}
View Source
var AServerAdvanceCommitIndex = distsys.MPCalArchetype{
	Name:              "AServerAdvanceCommitIndex",
	Label:             "AServerAdvanceCommitIndex.serverAdvanceCommitIndexLoop",
	RequiredRefParams: []string{"AServerAdvanceCommitIndex.net", "AServerAdvanceCommitIndex.netLen", "AServerAdvanceCommitIndex.netEnabled", "AServerAdvanceCommitIndex.fd", "AServerAdvanceCommitIndex.state", "AServerAdvanceCommitIndex.currentTerm", "AServerAdvanceCommitIndex.log", "AServerAdvanceCommitIndex.plog", "AServerAdvanceCommitIndex.commitIndex", "AServerAdvanceCommitIndex.nextIndex", "AServerAdvanceCommitIndex.matchIndex", "AServerAdvanceCommitIndex.votedFor", "AServerAdvanceCommitIndex.votesResponded", "AServerAdvanceCommitIndex.votesGranted", "AServerAdvanceCommitIndex.leader", "AServerAdvanceCommitIndex.sm", "AServerAdvanceCommitIndex.smDomain", "AServerAdvanceCommitIndex.leaderTimeout", "AServerAdvanceCommitIndex.appendEntriesCh", "AServerAdvanceCommitIndex.becomeLeaderCh"},
	RequiredValParams: []string{"AServerAdvanceCommitIndex.srvId"},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
		iface.EnsureArchetypeResourceLocal("AServerAdvanceCommitIndex.newCommitIndex", tla.MakeNumber(0))
	},
}
View Source
var AServerAppendEntries = distsys.MPCalArchetype{
	Name:              "AServerAppendEntries",
	Label:             "AServerAppendEntries.serverAppendEntriesLoop",
	RequiredRefParams: []string{"AServerAppendEntries.net", "AServerAppendEntries.netLen", "AServerAppendEntries.netEnabled", "AServerAppendEntries.fd", "AServerAppendEntries.state", "AServerAppendEntries.currentTerm", "AServerAppendEntries.log", "AServerAppendEntries.plog", "AServerAppendEntries.commitIndex", "AServerAppendEntries.nextIndex", "AServerAppendEntries.matchIndex", "AServerAppendEntries.votedFor", "AServerAppendEntries.votesResponded", "AServerAppendEntries.votesGranted", "AServerAppendEntries.leader", "AServerAppendEntries.sm", "AServerAppendEntries.smDomain", "AServerAppendEntries.leaderTimeout", "AServerAppendEntries.appendEntriesCh", "AServerAppendEntries.becomeLeaderCh"},
	RequiredValParams: []string{"AServerAppendEntries.srvId"},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
		iface.EnsureArchetypeResourceLocal("AServerAppendEntries.idx", tla.Value{})
	},
}
View Source
var AServerBecomeLeader = distsys.MPCalArchetype{
	Name:              "AServerBecomeLeader",
	Label:             "AServerBecomeLeader.serverBecomeLeaderLoop",
	RequiredRefParams: []string{"AServerBecomeLeader.net", "AServerBecomeLeader.netLen", "AServerBecomeLeader.netEnabled", "AServerBecomeLeader.fd", "AServerBecomeLeader.state", "AServerBecomeLeader.currentTerm", "AServerBecomeLeader.log", "AServerBecomeLeader.plog", "AServerBecomeLeader.commitIndex", "AServerBecomeLeader.nextIndex", "AServerBecomeLeader.matchIndex", "AServerBecomeLeader.votedFor", "AServerBecomeLeader.votesResponded", "AServerBecomeLeader.votesGranted", "AServerBecomeLeader.leader", "AServerBecomeLeader.sm", "AServerBecomeLeader.smDomain", "AServerBecomeLeader.leaderTimeout", "AServerBecomeLeader.appendEntriesCh", "AServerBecomeLeader.becomeLeaderCh"},
	RequiredValParams: []string{"AServerBecomeLeader.srvId"},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
	},
}
View Source
var AServerCrasher = distsys.MPCalArchetype{
	Name:              "AServerCrasher",
	Label:             "AServerCrasher.serverCrash",
	RequiredRefParams: []string{"AServerCrasher.netEnabled", "AServerCrasher.fd"},
	RequiredValParams: []string{"AServerCrasher.srvId"},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
	},
}
View Source
var AServerRequestVote = distsys.MPCalArchetype{
	Name:              "AServerRequestVote",
	Label:             "AServerRequestVote.serverRequestVoteLoop",
	RequiredRefParams: []string{"AServerRequestVote.net", "AServerRequestVote.netLen", "AServerRequestVote.netEnabled", "AServerRequestVote.fd", "AServerRequestVote.state", "AServerRequestVote.currentTerm", "AServerRequestVote.log", "AServerRequestVote.plog", "AServerRequestVote.commitIndex", "AServerRequestVote.nextIndex", "AServerRequestVote.matchIndex", "AServerRequestVote.votedFor", "AServerRequestVote.votesResponded", "AServerRequestVote.votesGranted", "AServerRequestVote.leader", "AServerRequestVote.sm", "AServerRequestVote.smDomain", "AServerRequestVote.leaderTimeout", "AServerRequestVote.appendEntriesCh", "AServerRequestVote.becomeLeaderCh"},
	RequiredValParams: []string{"AServerRequestVote.srvId"},
	JumpTable:         jumpTable,
	ProcTable:         procTable,
	PreAmble: func(iface distsys.ArchetypeInterface) {
		iface.EnsureArchetypeResourceLocal("AServerRequestVote.idx", tla.MakeNumber(1))
	},
}
View Source
var LeaderTimeoutConstantDefs = distsys.EnsureMPCalContextConfigs(
	distsys.DefineConstantValue("LeaderTimeoutReset", tla.MakeBool(true)),
)
View Source
var PersistentLogConstantDefs = distsys.EnsureMPCalContextConfigs(
	distsys.DefineConstantValue("LogConcat", logConcat),
	distsys.DefineConstantValue("LogPop", logPop),
)

Functions

func AppendEntriesRequest

func AppendEntriesRequest(iface distsys.ArchetypeInterface) tla.Value

func AppendEntriesResponse

func AppendEntriesResponse(iface distsys.ArchetypeInterface) tla.Value

func ApplyLog

func ApplyLog(iface distsys.ArchetypeInterface, xlog tla.Value, start tla.Value, end tla.Value, xsm0 tla.Value, xsmDomain0 tla.Value) tla.Value

func ApplyLogEntry

func ApplyLogEntry(iface distsys.ArchetypeInterface, xentry tla.Value, xsm tla.Value, xsmDomain tla.Value) tla.Value

func Candidate

func Candidate(iface distsys.ArchetypeInterface) tla.Value

func ClientGetRequest

func ClientGetRequest(iface distsys.ArchetypeInterface) tla.Value

func ClientGetResponse

func ClientGetResponse(iface distsys.ArchetypeInterface) tla.Value

func ClientPutRequest

func ClientPutRequest(iface distsys.ArchetypeInterface) tla.Value

func ClientPutResponse

func ClientPutResponse(iface distsys.ArchetypeInterface) tla.Value

func ClientSet

func ClientSet(iface distsys.ArchetypeInterface) tla.Value

func FindMaxAgreeIndex

func FindMaxAgreeIndex(iface distsys.ArchetypeInterface, logLocal tla.Value, i tla.Value, matchIndex tla.Value) tla.Value

func FindMaxAgreeIndexRec

func FindMaxAgreeIndexRec(iface distsys.ArchetypeInterface, logLocal0 tla.Value, i0 tla.Value, matchIndex0 tla.Value, index tla.Value) tla.Value

func Follower

func Follower(iface distsys.ArchetypeInterface) tla.Value

func Get

func IsQuorum

func IsQuorum(iface distsys.ArchetypeInterface, s3 tla.Value) tla.Value

func Key1

func Key1(iface distsys.ArchetypeInterface) tla.Value

func Key2

func Key2(iface distsys.ArchetypeInterface) tla.Value

func KeySet

func KeySet(iface distsys.ArchetypeInterface) tla.Value

func LastTerm

func LastTerm(iface distsys.ArchetypeInterface, xlog0 tla.Value) tla.Value

func Leader

func Leader(iface distsys.ArchetypeInterface) tla.Value

func Max

func MaxAcc

func MaxAcc(iface distsys.ArchetypeInterface, s2 tla.Value, e10 tla.Value) tla.Value

func Min

func MinAcc

func MinAcc(iface distsys.ArchetypeInterface, s0 tla.Value, e1 tla.Value) tla.Value

func NewCustomInChan

func NewCustomInChan(ch <-chan tla.Value, timeout time.Duration) distsys.ArchetypeResource

func NewPersistentLog

func NewPersistentLog(name string, db *badger.DB) distsys.ArchetypeResource

func NewTimerResource

func NewTimerResource(timeout time.Duration, offset time.Duration) distsys.ArchetypeResource

func Nil

func NodeSet

func NodeSet(iface distsys.ArchetypeInterface) tla.Value

func Put

func RequestVoteRequest

func RequestVoteRequest(iface distsys.ArchetypeInterface) tla.Value

func RequestVoteResponse

func RequestVoteResponse(iface distsys.ArchetypeInterface) tla.Value

func ServerAdvanceCommitIndexSet

func ServerAdvanceCommitIndexSet(iface distsys.ArchetypeInterface) tla.Value

func ServerAppendEntriesSet

func ServerAppendEntriesSet(iface distsys.ArchetypeInterface) tla.Value

func ServerBecomeLeaderSet

func ServerBecomeLeaderSet(iface distsys.ArchetypeInterface) tla.Value

func ServerCrasherSet

func ServerCrasherSet(iface distsys.ArchetypeInterface) tla.Value

func ServerRequestVoteSet

func ServerRequestVoteSet(iface distsys.ArchetypeInterface) tla.Value

func ServerSet

func ServerSet(iface distsys.ArchetypeInterface) tla.Value

func Value1

func Value1(iface distsys.ArchetypeInterface) tla.Value

Types

type CustomInChan

type CustomInChan struct {
	distsys.ArchetypeResourceLeafMixin
	// contains filtered or unexported fields
}

CustomInChan is similar resources.InputChannel, however, after a timeout it returns a default value instead of aborting the critical section. It used in implementing periodic sending of AppendEntries request. In some cases, the request should be sent immediately, for example, when the server just becomes a leader. In this case, the input channel signals.

func (*CustomInChan) Abort

func (res *CustomInChan) Abort() chan struct{}

func (*CustomInChan) Close

func (res *CustomInChan) Close() error

func (*CustomInChan) Commit

func (res *CustomInChan) Commit() chan struct{}

func (*CustomInChan) PreCommit

func (res *CustomInChan) PreCommit() chan error

func (*CustomInChan) ReadValue

func (res *CustomInChan) ReadValue() (tla.Value, error)

func (*CustomInChan) WriteValue

func (res *CustomInChan) WriteValue(value tla.Value) error

type ImmutableResource

type ImmutableResource struct {
	distsys.ArchetypeResourceLeafMixin
	// contains filtered or unexported fields
}

func (*ImmutableResource) Abort

func (res *ImmutableResource) Abort() chan struct{}

func (*ImmutableResource) Close

func (res *ImmutableResource) Close() error

func (*ImmutableResource) Commit

func (res *ImmutableResource) Commit() chan struct{}

func (*ImmutableResource) PreCommit

func (res *ImmutableResource) PreCommit() chan error

func (*ImmutableResource) ReadValue

func (res *ImmutableResource) ReadValue() (tla.Value, error)

func (*ImmutableResource) WriteValue

func (res *ImmutableResource) WriteValue(value tla.Value) error

type PersistentLog

type PersistentLog struct {
	// contains filtered or unexported fields
}

PersistentLog is a distsys.ArchetypeResource that implements Raft's persistent log behavior.

func (*PersistentLog) Abort

func (res *PersistentLog) Abort() chan struct{}

func (*PersistentLog) Close

func (res *PersistentLog) Close() error

func (*PersistentLog) Commit

func (res *PersistentLog) Commit() chan struct{}

func (*PersistentLog) Index

func (res *PersistentLog) Index(index tla.Value) (distsys.ArchetypeResource, error)

func (*PersistentLog) PreCommit

func (res *PersistentLog) PreCommit() chan error

func (*PersistentLog) ReadValue

func (res *PersistentLog) ReadValue() (tla.Value, error)

func (*PersistentLog) VClockHint

func (res *PersistentLog) VClockHint(archClock trace.VClock) trace.VClock

func (*PersistentLog) WriteValue

func (res *PersistentLog) WriteValue(value tla.Value) error

type TimerResource

type TimerResource struct {
	distsys.ArchetypeResourceLeafMixin
	// contains filtered or unexported fields
}

TimerResource is used to implement randomized timeout in the Raft leader election. It measures the time since the last call to Read and if it's greater than some random timeout, then it returns true; otherwise, returns false. Also, it supports timer reset through write calls.

func (*TimerResource) Abort

func (res *TimerResource) Abort() chan struct{}

func (*TimerResource) Close

func (res *TimerResource) Close() error

func (*TimerResource) Commit

func (res *TimerResource) Commit() chan struct{}

func (*TimerResource) PreCommit

func (res *TimerResource) PreCommit() chan error

func (*TimerResource) ReadValue

func (res *TimerResource) ReadValue() (tla.Value, error)

func (*TimerResource) WriteValue

func (res *TimerResource) WriteValue(value tla.Value) error

Directories

Path Synopsis
cmd

Jump to

Keyboard shortcuts

? : This menu
/ : Search site
f or F : Jump to
y or Y : Canonical URL