raft

package module
v0.0.0-...-e2172d6 Latest Latest
Warning

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

Go to latest
Published: Jul 23, 2024 License: MIT Imports: 18 Imported by: 0

README

GoDoc Go Report Card Coverage Status CircleCI

Overview

Raft is a protocol with which a cluster of nodes can maintain a replicated state machine. The state machine is kept in sync through the use of a replicated log. However, The details of the Raft protocol are outside the scope of this document, For more details on Raft, see In Search of an Understandable Consensus Algorithm

Why another library?

Raft algorithm comes in search of an understandable consensus algorithm, unfortunately, most of the go libraries out there required a deep knowledge of their implementation and APIs.

This raft library was born to align with the understandability raft principle and its sole purpose is to provide consensus with the minimalistic, simple, clean, and idiomatic API.

Etcd Raft is the most widely used Raft library in production But, it follows a minimalistic design philosophy by only implementing the core raft algorithm which leaves gaps and ambiguities.

So, instead of reinventing the wheel, this library uses etcd raft as its core.

That's how you can benefit from the power and stability of etcd raft, with an understandable API. indeed, it keeps your focus on building awesome software.

Finally, the raft library aimed to be used internally but it is worth being exposed to the public.

Features

This raft implementation is a full feature implementation of Raft protocol. Features includes:

  • Mange Multi-Raft
  • Coalesced heartbeats to reduce the overhead of heartbeats when there are a large number of raft groups
  • Leader election
  • Log replication
  • Log compaction
  • Pre-Vote Protocol
  • Membership changes
    • add member
    • remove member
    • update member
    • promote member
    • demote member
  • Leadership transfer extension
  • Efficient linearizable read-only queries served by both the leader and followers
    • leader checks with quorum and bypasses Raft log before processing read-only queries
    • followers asks leader to get a safe read index before processing read-only queries
  • More efficient lease-based linearizable read-only queries served by both the leader and followers
    • leader bypasses Raft log and processing read-only queries locally
    • followers asks leader to get a safe read index before processing read-only queries
    • this approach relies on the clock of the all the machines in raft group
  • Snapshots
    • automatic snapshots when the log store reaches a certain length
    • API to force new snapshot
  • Read-Only Members
    • learner member
    • staging member
  • Segmented WAL to provide durability and ensure data integrity
  • Garbage collector to controls how many WAL and snapshot files are retained
  • Network transport to communicate with Raft on remote machines
    • gRPC (recommended)
    • http
  • gRPC chunked transfer encoding
  • Network Pipelining
  • Restore cluster quorum and data
    • force new cluster from the existing WALL and snapshot
    • restore the cluster from snapshot
  • Optimistic pipelining to reduce log replication latency
  • Flow control for log replication
  • Batching Raft messages to reduce synchronized network I/O calls
  • Batching log entries to reduce disk synchronized I/O
  • Writing to leader's disk in parallel
  • Internal proposal redirection from followers to leader
  • Automatic stepping down when the leader loses quorum
  • Protection against unbounded log growth when quorum is lost

WAL's and snapshots

There are two sets of files on disk that provide persistent state for Raft. There is a set of WAL (write-ahead log files). These store a series of log entries and Raft metadata, such as the current term, index, and committed index. WAL files are automatically rotated when they reach a certain size.

To avoid having to retain every entry in the history of the log, snapshots serialize a view of the state at a particular point in time. After a snapshot gets taken, logs that predate the snapshot are no longer necessary, because the snapshot captures all the information that's needed from the log up to that point. The number of old snapshots and WALs to retain is configurable.

WALs mostly contain protobuf-serialized user data store modifications. A log entry can contain a batch of creations, updates, and deletions of objects from the user data store. Some log entries contain other kinds of metadata, like node additions or removals. Snapshots contain a complete dump of the store, as well as any metadata from the log entries that needs to be preserved. The saved metadata includes the Raft term and index, a list of nodes in the cluster, and a list of nodes that have been removed from the cluster.

Raft IDs

The library uses integers to identify Raft nodes. The Raft IDs may assigned dynamically when a node joins the Raft consensus group, or it can be defined manually by the user.

It's important to note that a Raft ID can't be reused after a node that was using the ID leaves the consensus group. These Raft IDs of nodes that are no longer part of the cluster are saved (persisted on disk) as part of the nodes pool members to make sure they aren't reused. If a node with a removed Raft ID tries to use Raft RPCs, other nodes won't honor these requests.

The removed node's IDs are used to restrict these nodes from communicating, affecting the cluster state and avoid ambiguity.

Initializing a Raft cluster

The first member of a cluster assigns itself a random Raft ID unless it pre-defined. It creates a new WAL with its own Raft identity stored in the metadata field. The metadata field is the only part of the WAL that differs between nodes. By storing information such as the local Raft ID, it's easy to restore this node-specific information after a restart. In principle it could be stored in a separate file, but embedding it inside the WAL is most convenient.

The node then starts the Raft state machine. From this point, it's a fully functional single-node Raft instance. Writes to the data store actually go through Raft, though this is a trivial case because reaching consensus doesn't involve communicating with any other nodes.

Joining a Raft cluster

New nodes can join an existing Raft consensus group by invoking the Join RPC on any Raft member if proposal forwarding is enabled, Otherwise, Join RPC must be invoked on the leader. If successful, Join returns a Raft ID for the new node and a list of other members of the consensus group.

On the leader side, Join tries to append a configuration change entry to the Raft log, and waits until that entry becomes committed.

A new node creates an empty Raft log with its own node information in the metadata field. Then it starts the state machine. By running the Raft consensus protocol, the leader will discover that the new node doesn't have any entries in its log, and will synchronize these entries to the new node through some combination of sending snapshots and log entries. It can take a little while for a new node to become a functional member of the consensus group, because it needs to receive this data first.

The new node can join the cluster as a Voter, Learner, or Staging member.

Initializing a predefined Raft cluster

The library also provides a mechanism to initialize and boot the Raft cluster from a predefined members configuration. This is done by applying the same configurations to all members even for the later joining member.

Each member of the cluster use its predefined id and creates a new WAL with its own Raft identity stored in the metadata field. The member node then starts the Raft state machine.

Once all members nodes are started the election process will be initiated, and when all members agrees on the same leader the cluster becomes fully functional, Writes to the data store actually go through Raft and it considered complete after reaching a majority.

Usage

The primary object in raft is a Node. Either start a Node from scratch using raft.WithInitCluster(), raft.WithJoin() or start a Node from some initial state using raft.WithRestart().

To start a three-node cluster from predefined configuration:

Node A

m1 := raft.RawMember{ID: 1, Address: ":8081"}
m2 := raft.RawMember{ID: 2, Address: ":8082"}
m3 := raft.RawMember{ID: 3, Address: ":8083"}
node := raft.NewNode(<FSM>, <Transport>, <Opts>)
// The first member should reference to the current effective member.
node.Start(raft.WithInitCluster(), raft.WithMembers(m1, m2, m3))

Node B

m1 := raft.RawMember{ID: 1, Address: ":8081"}
m2 := raft.RawMember{ID: 2, Address: ":8082"}
m3 := raft.RawMember{ID: 3, Address: ":8083"}
node := raft.NewNode(<FSM>, <Transport>, <Opts>)
// The first member should reference to the current effective member.
node.Start(raft.WithInitCluster(), raft.WithMembers(m2, m1, m3))

Node C

m1 := raft.RawMember{ID: 1, Address: ":8081"}
m2 := raft.RawMember{ID: 2, Address: ":8082"}
m3 := raft.RawMember{ID: 3, Address: ":8083"}
node := raft.NewNode(<FSM>, <Transport>, <Opts>)
// The first member should reference to the current effective member.
node.Start(raft.WithInitCluster(), raft.WithMembers(m3, m1, m2))

Start a single node cluster, like so:

m := raft.RawMember{ID: 1, Address: ":8081"}
opt := raft.WithMembers(m)
// or 
opt = raft.WithAddress(":8081")
node := raft.NewNode(<FSM>, <Transport>, <Opts>)
node.Start(raft.WithInitCluster(), opt)

To allow a new node to join a cluster, like so:

m := raft.RawMember{ID: 2, Address: ":8082"}
opt := raft.WithMembers(m)
// or 
opt = raft.WithAddress(":8082")
node := raft.NewNode(<FSM>, <Transport>, <Opts>)
node.Start(raft.WithJoin(":8081", time.Second), opt)

To restart a node from previous state:

node := raft.NewNode(<FSM>, <Transport>, <Opts>)
node.Start(raft.WithRestart())

To force new cluster:

node := raft.NewNode(<FSM>, <Transport>, <Opts>)
// This will use the latest wal and snapshot.
node.Start(raft.WithForceNewCluster())

To restore from snapshot:

node := raft.NewNode(<FSM>, <Transport>, <Opts>)
node.Start(raft.WithRestore("<path to snapshot file>"))

Examples and dcos

  • More detailed development documentation can be found in go docs
  • Fully working single and multiraft cluster example can be found in Examples Folder.

Contributing to this project

We welcome contributions. If you find any bugs, potential flaws and edge cases, improvements, new feature suggestions or discussions, please submit issues or pull requests.

Documentation

Overview

Example (GRPC)
package main

import (
	"io"

	"github.com/alekseyvit/raft"
	"github.com/alekseyvit/raft/transport"
	"github.com/alekseyvit/raft/transport/raftgrpc"
	"google.golang.org/grpc"
)

type stateMachine struct{}

func (stateMachine) Apply([]byte)                           {}
func (stateMachine) Snapshot() (r io.ReadCloser, err error) { return }
func (stateMachine) Restore(io.ReadCloser) (err error)      { return }

func main() {
	srv := grpc.NewServer()
	node := raft.NewNode(stateMachine{}, transport.GRPC)
	raftgrpc.RegisterHandler(srv, node.Handler())
}
Output:

Example (Http)
package main

import (
	"io"
	"net/http"

	"github.com/alekseyvit/raft"
	"github.com/alekseyvit/raft/transport"
	"github.com/alekseyvit/raft/transport/rafthttp"
)

type stateMachine struct{}

func (stateMachine) Apply([]byte)                           {}
func (stateMachine) Snapshot() (r io.ReadCloser, err error) { return }
func (stateMachine) Restore(io.ReadCloser) (err error)      { return }

func main() {
	node := raft.NewNode(stateMachine{}, transport.HTTP)
	handler := rafthttp.Handler(node.Handler())
	_ = http.Server{
		Handler: handler,
	}
}
Output:

Index

Examples

Constants

View Source
const None = raft.None

None is a placeholder node ID used to identify non-existence.

Variables

View Source
var (
	// ErrNodeStopped is returned by the Node methods after a call to
	// Shutdown or when it has not started.
	ErrNodeStopped = raftengine.ErrStopped
	// ErrNotLeader is returned when an operation can't be completed on a
	// follower or candidate node
	ErrNotLeader = errors.New("raft: node is not the leader")
)

Functions

This section is empty.

Types

type Member

type Member interface {
	ID() uint64
	Address() string
	ActiveSince() time.Time
	IsActive() bool
	Type() MemberType
	Raw() RawMember
}

Member represents a raft cluster member.

type MemberType

type MemberType = raftpb.MemberType

MemberType used to distinguish members (voter, learner, etc).

const (
	// VoterMember participate in elections and log entry commitment, It is the default type.
	VoterMember MemberType = raftpb.VoterMember
	// RemovedMember represents an removed raft node.
	RemovedMember MemberType = raftpb.RemovedMember
	// LearnerMember will receive log entries, but it won't participate in elections or log entry commitment.
	LearnerMember MemberType = raftpb.LearnerMember
	// StagingMember will receive log entries, but it won't participate in elections or log entry commitment,
	// and once it receives enough log entries to be sufficiently caught up to
	// the leader's log, the leader will promote him to VoterMember.
	StagingMember MemberType = raftpb.StagingMember
)

type Node

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

Node is a controller of the current effective raft member, It also represents the front API to proposes changes into the raft cluster.

Node packed with a built-in segmented WAL to provide durability and ensure data integrity. alongside snapshotter that take a snapshot of the state of a system at a particular point in time. Although, the application must have its own backend DB delegated by the state machine interface.

Node also maintains a membership pool containing all other raft members.

Multiple goroutines may invoke methods on a Node simultaneously.

func NewNode

func NewNode(fsm StateMachine, proto etransport.Proto, opts ...Option) *Node

NewNode construct a new node from the given configuration. The returned node is in a stopped state, therefore it must be start explicitly.

func (*Node) AddMember

func (n *Node) AddMember(ctx context.Context, raw *RawMember) error

AddMember proposes to add the given member to the cluster, It considered complete after reaching a majority. After committing the addition, each member in the cluster add the given member to its pool.

Although, most applications will use the basic join.

If the provided context expires before, the add is complete, AddMember returns the context's error, otherwise it returns any error returned due to the add.

If the provided member id is None, AddMember will assign next available id.

func (*Node) DemoteMember

func (n *Node) DemoteMember(ctx context.Context, id uint64) error

DemoteMember proposes to take away a member vote. It considered complete after reaching a majority. After committing the demotion, each member in the cluster updates the given member type on its pool.

If the provided context expires before, the promotion is complete, DemoteMember returns the context's error, otherwise it returns any error returned due to the demotion.

func (*Node) GetMemebr

func (n *Node) GetMemebr(id uint64) (Member, bool)

GetMemebr returns member associated to the given id if exist, Otherwise, it return nil and false.

func (*Node) Handler

func (n *Node) Handler() etransport.Handler

Handler return node transportation handler, that delegated to respond to RPC requests over the wire. the returned handler must be registered with the transportation server, unless the node is registered with a node group.

func (*Node) HardState

func (n *Node) HardState() (etcdraftpb.HardState, error)

func (*Node) Leader

func (n *Node) Leader() uint64

Leader returns the id of the raft cluster leader, if there any. Otherwise, it return None.

func (*Node) Leave

func (n *Node) Leave(ctx context.Context) error

Leave proposes to remove current effective member. See the documentation of "RemoveMember" for more information.

func (*Node) LinearizableRead

func (n *Node) LinearizableRead(ctx context.Context) error

LinearizableRead implies that once a write completes, all later reads should return the value of that write, or the value of a later write.

func (*Node) Members

func (n *Node) Members() []Member

Members returns the list of raft Members in the Cluster.

func (*Node) PromoteMember

func (n *Node) PromoteMember(ctx context.Context, id uint64) error

PromoteMember proposes to promote a learner member to a voting member, It considered complete after reaching a majority. After committing the promotion, each member in the cluster updates the given member type on its pool.

If the provided context expires before, the promotion is complete, PromoteMember returns the context's error, otherwise it returns any error returned due to the promotion.

func (*Node) RemoveMember

func (n *Node) RemoveMember(ctx context.Context, id uint64) error

RemoveMember proposes to remove the given member from the cluster, It considered complete after reaching a majority. After committing the removal, each member in the cluster remove the given member from its pool.

Although, the removed member configuration will remain and only its type will be changed to "RemovedMember". therefore its id is not reusable again, and its cannot rejoin the cluster again.

If the provided context expires before, the removal is complete, RemoveMember returns the context's error, otherwise it returns any error returned due to the removal.

func (*Node) Replicate

func (n *Node) Replicate(ctx context.Context, data []byte) error

Replicate proposes to replicate the given data to all raft members, in a highly consistent manner. If the provided context expires before, the replication is complete, Replicate returns the context's error, otherwise it returns any error returned due to the replication.

func (*Node) Shutdown

func (n *Node) Shutdown(ctx context.Context) error

Shutdown gracefully shuts down the node without interrupting any active requests. Shutdown works by first closing all open requests listeners, then blocks until all the pending requests are finished, and then shut down. If the provided context expires before the shutdown is complete, Shutdown force the node to shut off, Shutdown returns any error returned from closing the Node's underlying internal(s).

When Shutdown is called, Start may immediately return ErrNodeStopped. Make sure the program doesn't exit and waits instead for Shutdown to return.

func (*Node) Snapshot

func (n *Node) Snapshot() (io.ReadCloser, error)

Snapshot is used to manually force node to take a snapshot. Returns a io.ReadCloser that can be used to to read snapshot file. the caller must invoke close method on the returned io.ReadCloser explicitly, Otherwise, the underlying os.File remain open.

func (*Node) SoftStatus

func (n *Node) SoftStatus() (raft.SoftState, error)

It is not required to consume or store SoftState.

func (*Node) Start

func (n *Node) Start(opts ...StartOption) error

Start start the node and accepts incoming requests on the handler or on local node methods. It can be called after Stop to restart the node.

Start always returns a non-nil error. After Shutdown, the returned error is ErrNodeStopped.

func (*Node) Stepdown

func (n *Node) Stepdown(ctx context.Context) error

Stepdown proposes to transfer leadership to the longest active member in the cluster. This must be run on the leader or it will fail.

func (*Node) TermIndex

func (n *Node) TermIndex() (uint64, uint64, error)

func (*Node) TransferLeadership

func (n *Node) TransferLeadership(ctx context.Context, id uint64) error

TransferLeadership proposes to transfer leadership to the given member id.

func (*Node) UpdateMember

func (n *Node) UpdateMember(ctx context.Context, raw *RawMember) error

UpdateMember proposes to update the given member, It considered complete after reaching a majority. After committing the update, each member in the cluster updates the given member configuration on its pool.

If the provided context expires before, the update is complete, UpdateMember returns the context's error, otherwise it returns any error returned due to the update.

Note: the member id and type are not updatable.

func (*Node) Whoami

func (n *Node) Whoami() uint64

Whoami returns the id associated with current effective member. It return None, if the node stopped or not yet part of a raft cluster.

type NodeGroup

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

NodeGroup manage multi Raft nodes from many different Raft groups known as Raft clusters. NodeGroup is more efficient than a collection of nodes.

Scales raft into multiple raft groups requires data sharding, each raft group responsible for managing data in the range [start, end]. as the system grows to include more ranges, so does the amount of traffic required to handle heartbeats. The number of ranges is much larger than the number of physical nodes so many ranges will have overlapping membership this is where NodeGroup comes in: instead of allowing each range to run Raft independently, we manage an entire node’s worth of ranges as a group. Each pair of physical nodes only needs to exchange heartbeats once per tick (coalesced heartbeats), no matter how many ranges they have in common.

Create, Remove can run while node group stopped. starting an created node is required a started node group, Otherwise, it will hang until the node group started.

func NewNodeGroup

func NewNodeGroup(proto etransport.Proto) *NodeGroup

NewNodeGroup returns a new NodeGroup. the returned node group will lazily initialize, from the first node registered within it, So it's recommended to apply the same HeartbeatTick, ElectionTick, and TickInterval configuration to all sub-nodes.

func (*NodeGroup) Create

func (ng *NodeGroup) Create(groupID uint64, fsm StateMachine, opts ...Option) *Node

Create construct and returns a new node that associated to the given group id,

The node and the group are correlated so each group id must have its own node object and each node object must have its own group id.

All registered nodes within the node group must have the same id, that is how multiple nodes object representing one single physical node that participate in multiple raft groups. Starting a node with a different id from the previous one will cause a panic. Make sure the program set the node id using option, if it's not first node.

func (*NodeGroup) Handler

func (ng *NodeGroup) Handler() etransport.Handler

Handler return NodeGroup transportation handler, that delegated to respond to RPC requests over the wire. the returned handler must be registered with the transportation server.

func (*NodeGroup) Remove

func (ng *NodeGroup) Remove(groupID uint64)

Remove remove node related to the given group id. after the removal, the actual node will become idle, it must coordinate with node shutdown explicitly.

nodeGroup.Remove(12)
node.Shutdown(ctx)

func (*NodeGroup) Start

func (ng *NodeGroup) Start()

Start starts the NodeGroup. It can be called after Stop to restart the NodeGroup. Start returns when Stop called.

func (*NodeGroup) Stop

func (ng *NodeGroup) Stop()

Stop performs any necessary termination of the NodeGroup.

type Option

type Option interface {
	// contains filtered or unexported methods
}

Option configures raft node using the functional options paradigm popularized by Rob Pike and Dave Cheney. If you're unfamiliar with this style, see https://commandcenter.blogspot.com/2014/01/self-referential-functions-and-design.html and https://dave.cheney.net/2014/10/17/functional-options-for-friendly-apis.

func WithCheckQuorum

func WithCheckQuorum() Option

WithCheckQuorum specifies if the leader should check quorum activity. Leader steps down when quorum is not active for an electionTimeout.

Default Value: false.

func WithContext

func WithContext(ctx context.Context) Option

WithContext set raft node parent ctx, The provided ctx must be non-nil.

The context controls the entire lifetime of the raft node: obtaining a connection, sending the msgs, reading the response, and process msgs.

Default Value: context.Background().

func WithDisableProposalForwarding

func WithDisableProposalForwarding() Option

WithDisableProposalForwarding set to true means that followers will drop proposals, rather than forwarding them to the leader. One use case for this feature would be in a situation where the Raft leader is used to compute the data of a proposal, for example, adding a timestamp from a hybrid logical clock to data in a monotonically increasing way. Forwarding should be disabled to prevent a follower with an inaccurate hybrid logical clock from assigning the timestamp and then forwarding the data to the leader.

Default Value: false.

func WithDrainTimeOut

func WithDrainTimeOut(d time.Duration) Option

WithDrainTimeOut is the timeout on the streaming pending messages to other raft members. Drain can be very useful for graceful shutdown.

Default Value: 10's.

func WithElectionTick

func WithElectionTick(tick int) Option

WithElectionTick is the number of node tick (WithTickInterval) invocations that must pass between elections. That is, if a follower does not receive any message from the leader of current term before ElectionTick has elapsed, it will become candidate and start an election. ElectionTick must be greater than HeartbeatTick. We suggest ElectionTick = 10 * HeartbeatTick to avoid unnecessary leader switching.

Default Value: 10.

func WithHeartbeatTick

func WithHeartbeatTick(tick int) Option

WithHeartbeatTick is the number of node tick (WithTickInterval) invocations that

must pass between heartbeats. That is, a leader sends heartbeat messages to

maintain its leadership every HeartbeatTick ticks.

Default Value: 1.

func WithLinearizableReadLeaseBased

func WithLinearizableReadLeaseBased() Option

WithLinearizableReadLeaseBased ensures linearizability of the read only request by relying on the leader lease. It can be affected by clock drift. If the clock drift is unbounded, leader might keep the lease longer than it should (clock can move backward/pause without any bound). ReadIndex is not safe in that case.

func WithLinearizableReadSafe

func WithLinearizableReadSafe() Option

WithLinearizableReadSafe guarantees the linearizability of the read request by communicating with the quorum. It is the default and suggested option.

func WithLogger

func WithLogger(lg raftlog.Logger) Option

WithLogger sets logger that is used to generates lines of output.

Default Value: raftlog.DefaultLogger.

func WithMaxCommittedSizePerReady

func WithMaxCommittedSizePerReady(max uint64) Option

WithMaxCommittedSizePerReady limits the size of the committed entries which can be applied.

Default Value: 0.

func WithMaxInflightMsgs

func WithMaxInflightMsgs(max int) Option

WithMaxInflightMsgs limits the max number of in-flight append messages during optimistic replication phase. The application transportation layer usually has its own sending buffer over TCP/UDP. Setting MaxInflightMsgs to avoid overflowing that sending buffer.

Default Value: 256.

func WithMaxSizePerMsg

func WithMaxSizePerMsg(max uint64) Option

WithMaxSizePerMsg limits the max byte size of each append message. Smaller value lowers the raft recovery cost(initial probing and message lost during normal operation). On the other side, it might affect the throughput during normal replication. Note: math.MaxUint64 for unlimited, 0 for at most one entry per message.

Default Value: 1024 * 1024.

func WithMaxSnapshotFiles

func WithMaxSnapshotFiles(max int) Option

WithMaxSnapshotFiles is the number of snapshots to keep beyond the current snapshot.

Default Value: 5.

func WithMaxUncommittedEntriesSize

func WithMaxUncommittedEntriesSize(max uint64) Option

WithMaxUncommittedEntriesSize limits the aggregate byte size of the uncommitted entries that may be appended to a leader's log. Once this limit is exceeded, proposals will begin to return ErrProposalDropped errors. Note: 0 for no limit.

Default Value: 1 << 30.

func WithPipelining

func WithPipelining() Option

WithPipelining is the process to send successive requests, over the same persistent connection, without waiting for the answer. This avoids latency of the connection. Theoretically, performance could also be improved if two or more requests were to be packed into the same connection.

Note: pipelining spawn 4 goroutines per remote member connection.

func WithPreVote

func WithPreVote() Option

WithPreVote enables the Pre-Vote algorithm described in raft thesis section 9.6. This prevents disruption when a node that has been partitioned away rejoins the cluster.

Default Value: false.

func WithSnapshotInterval

func WithSnapshotInterval(i uint64) Option

WithSnapshotInterval is the number of log entries between snapshots.

Default Value: 1000.

func WithStateDIR

func WithStateDIR(dir string) Option

WithStateDIR is the directory to store durable state (WAL logs and Snapshots).

Default Value: os.TempDir().

func WithStreamTimeOut

func WithStreamTimeOut(d time.Duration) Option

WithStreamTimeOut is the timeout on the streaming messages to other raft members.

Default Value: 10's.

func WithTickInterval

func WithTickInterval(d time.Duration) Option

WithTickInterval is the time interval to, increments the internal logical clock for, the current raft member by a single tick.

Default Value: 100'ms.

type RawMember

type RawMember = raftpb.Member

RawMember represents a raft cluster member and holds its metadata.

type StartOption

type StartOption interface {
	// contains filtered or unexported methods
}

StartOption configures how we start the raft node using the functional options paradigm , popularized by Rob Pike and Dave Cheney. If you're unfamiliar with this style, see https://commandcenter.blogspot.com/2014/01/self-referential-functions-and-design.html and https://dave.cheney.net/2014/10/17/functional-options-for-friendly-apis.

func WithAddress

func WithAddress(addr string) StartOption

WithAddress set the raft node address.

func WithFallback

func WithFallback(opts ...StartOption) StartOption

WithFallback can be used if other options do not succeed.

WithFallback(
	WithJoin(),
	WithRestart,
)

func WithForceJoin

func WithForceJoin(addr string, timeout time.Duration) StartOption

WithForceJoin send rpc request to join an existing cluster even if already part of a cluster.

func WithForceNewCluster

func WithForceNewCluster() StartOption

WithForceNewCluster initialize a new cluster from state dir. One use case for this feature would be in restoring cluster quorum.

Note: ForceNewCluster preserve the same node id.

func WithInitCluster

func WithInitCluster() StartOption

WithInitCluster initialize a new cluster and create first raft node.

func WithJoin

func WithJoin(addr string, timeout time.Duration) StartOption

WithJoin send rpc request to join an existing cluster.

func WithMembers

func WithMembers(membs ...RawMember) StartOption

WithMembers add the given members to the raft node.

WithMembers safe to be used with initiate cluster kind options, ("WithForceNewCluster", "WithRestore", "WithInitCluster") Otherwise, it may conflicts with other options like WithJoin.

As long as only one url member, WithMembers will only set the current node, then it will be safe to be composed with other options even "WithJoin".

WithMembers and WithInitCluster must be applied to all cluster nodes when they are composed, Otherwise, the quorum will be lost and the cluster become unavailable.

Node A:
n.Start(WithInitCluster(), WithMembers(<node A>, <node B>))

Node B:
n.Start(WithInitCluster(), WithMembers(<node B>, <node A>))

Note: first member will be assigned to the current node.

func WithRestart

func WithRestart() StartOption

WithRestart restart raft node from state dir.

func WithRestore

func WithRestore(path string) StartOption

WithRestore initialize a new cluster from snapshot file. One use case for this feature would be in restoring cluster data.

type StateMachine

type StateMachine = raftengine.StateMachine

StateMachine define an interface that must be implemented by application to make use of the raft replicated log.

Directories

Path Synopsis
_examples
internal
mocks/membership
Package membershipmock is a generated GoMock package.
Package membershipmock is a generated GoMock package.
mocks/raftengine
Package raftenginemock is a generated GoMock package.
Package raftenginemock is a generated GoMock package.
mocks/storage
Package storagemock is a generated GoMock package.
Package storagemock is a generated GoMock package.
mocks/transport
Package transportmock is a generated GoMock package.
Package transportmock is a generated GoMock package.
Package raftlog implements a simple logging package.
Package raftlog implements a simple logging package.
Package rafttest provides functional tests for raft implementation.
Package rafttest provides functional tests for raft implementation.
Package transport provides types for raft transport functions.
Package transport provides types for raft transport functions.
raftgrpc
Package raftgrpc implements gRPC transportation layer for raft.
Package raftgrpc implements gRPC transportation layer for raft.
rafthttp
Package rafthttp implements HTTP transportation layer for raft.
Package rafthttp implements HTTP transportation layer for raft.

Jump to

Keyboard shortcuts

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