1package raft
2
3import (
4	"io"
5	"time"
6)
7
8// RPCResponse captures both a response and a potential error.
9type RPCResponse struct {
10	Response interface{}
11	Error    error
12}
13
14// RPC has a command, and provides a response mechanism.
15type RPC struct {
16	Command  interface{}
17	Reader   io.Reader // Set only for InstallSnapshot
18	RespChan chan<- RPCResponse
19}
20
21// Respond is used to respond with a response, error or both
22func (r *RPC) Respond(resp interface{}, err error) {
23	r.RespChan <- RPCResponse{resp, err}
24}
25
26// Transport provides an interface for network transports
27// to allow Raft to communicate with other nodes.
28type Transport interface {
29	// Consumer returns a channel that can be used to
30	// consume and respond to RPC requests.
31	Consumer() <-chan RPC
32
33	// LocalAddr is used to return our local address to distinguish from our peers.
34	LocalAddr() ServerAddress
35
36	// AppendEntriesPipeline returns an interface that can be used to pipeline
37	// AppendEntries requests.
38	AppendEntriesPipeline(id ServerID, target ServerAddress) (AppendPipeline, error)
39
40	// AppendEntries sends the appropriate RPC to the target node.
41	AppendEntries(id ServerID, target ServerAddress, args *AppendEntriesRequest, resp *AppendEntriesResponse) error
42
43	// RequestVote sends the appropriate RPC to the target node.
44	RequestVote(id ServerID, target ServerAddress, args *RequestVoteRequest, resp *RequestVoteResponse) error
45
46	// InstallSnapshot is used to push a snapshot down to a follower. The data is read from
47	// the ReadCloser and streamed to the client.
48	InstallSnapshot(id ServerID, target ServerAddress, args *InstallSnapshotRequest, resp *InstallSnapshotResponse, data io.Reader) error
49
50	// EncodePeer is used to serialize a peer's address.
51	EncodePeer(id ServerID, addr ServerAddress) []byte
52
53	// DecodePeer is used to deserialize a peer's address.
54	DecodePeer([]byte) ServerAddress
55
56	// SetHeartbeatHandler is used to setup a heartbeat handler
57	// as a fast-pass. This is to avoid head-of-line blocking from
58	// disk IO. If a Transport does not support this, it can simply
59	// ignore the call, and push the heartbeat onto the Consumer channel.
60	SetHeartbeatHandler(cb func(rpc RPC))
61}
62
63// WithClose is an interface that a transport may provide which
64// allows a transport to be shut down cleanly when a Raft instance
65// shuts down.
66//
67// It is defined separately from Transport as unfortunately it wasn't in the
68// original interface specification.
69type WithClose interface {
70	// Close permanently closes a transport, stopping
71	// any associated goroutines and freeing other resources.
72	Close() error
73}
74
75// LoopbackTransport is an interface that provides a loopback transport suitable for testing
76// e.g. InmemTransport. It's there so we don't have to rewrite tests.
77type LoopbackTransport interface {
78	Transport // Embedded transport reference
79	WithPeers // Embedded peer management
80	WithClose // with a close routine
81}
82
83// WithPeers is an interface that a transport may provide which allows for connection and
84// disconnection. Unless the transport is a loopback transport, the transport specified to
85// "Connect" is likely to be nil.
86type WithPeers interface {
87	Connect(peer ServerAddress, t Transport) // Connect a peer
88	Disconnect(peer ServerAddress)           // Disconnect a given peer
89	DisconnectAll()                          // Disconnect all peers, possibly to reconnect them later
90}
91
92// AppendPipeline is used for pipelining AppendEntries requests. It is used
93// to increase the replication throughput by masking latency and better
94// utilizing bandwidth.
95type AppendPipeline interface {
96	// AppendEntries is used to add another request to the pipeline.
97	// The send may block which is an effective form of back-pressure.
98	AppendEntries(args *AppendEntriesRequest, resp *AppendEntriesResponse) (AppendFuture, error)
99
100	// Consumer returns a channel that can be used to consume
101	// response futures when they are ready.
102	Consumer() <-chan AppendFuture
103
104	// Close closes the pipeline and cancels all inflight RPCs
105	Close() error
106}
107
108// AppendFuture is used to return information about a pipelined AppendEntries request.
109type AppendFuture interface {
110	Future
111
112	// Start returns the time that the append request was started.
113	// It is always OK to call this method.
114	Start() time.Time
115
116	// Request holds the parameters of the AppendEntries call.
117	// It is always OK to call this method.
118	Request() *AppendEntriesRequest
119
120	// Response holds the results of the AppendEntries call.
121	// This method must only be called after the Error
122	// method returns, and will only be valid on success.
123	Response() *AppendEntriesResponse
124}
125