etcd/raft/node.go

440 lines
12 KiB
Go
Raw Normal View History

/*
Copyright 2014 CoreOS, Inc.
Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
See the License for the specific language governing permissions and
limitations under the License.
*/
package raft
import (
2014-09-01 05:49:07 +04:00
"errors"
2014-09-03 07:21:58 +04:00
"log"
2014-09-01 05:49:07 +04:00
"github.com/coreos/etcd/Godeps/_workspace/src/golang.org/x/net/context"
2014-08-28 05:53:18 +04:00
pb "github.com/coreos/etcd/raft/raftpb"
)
2014-06-05 21:49:34 +04:00
2014-09-09 08:15:35 +04:00
var (
2014-09-16 04:35:02 +04:00
emptyState = pb.HardState{}
2014-10-08 02:22:35 +04:00
// ErrStopped is returned by methods on Nodes that have been stopped.
2014-09-09 08:15:35 +04:00
ErrStopped = errors.New("raft: stopped")
)
2014-09-01 05:49:07 +04:00
2014-09-16 04:35:02 +04:00
// SoftState provides state that is useful for logging and debugging.
// The state is volatile and does not need to be persisted to the WAL.
type SoftState struct {
Lead uint64
RaftState StateType
2014-09-16 04:35:02 +04:00
}
func (a *SoftState) equal(b *SoftState) bool {
2014-12-09 08:17:04 +03:00
return a.Lead == b.Lead && a.RaftState == b.RaftState
2014-09-16 04:35:02 +04:00
}
// Ready encapsulates the entries and messages that are ready to read,
// be saved to stable storage, committed or sent to other peers.
// All fields in Ready are read-only.
type Ready struct {
2014-09-16 04:35:02 +04:00
// The current volatile state of a Node.
// SoftState will be nil if there is no update.
// It is not required to consume or store SoftState.
*SoftState
// The current state of a Node to be saved to stable storage BEFORE
// Messages are sent.
// HardState will be equal to empty state if there is no update.
pb.HardState
// Entries specifies entries to be saved to stable storage BEFORE
// Messages are sent.
2014-08-28 05:53:18 +04:00
Entries []pb.Entry
2014-09-17 05:18:45 +04:00
// Snapshot specifies the snapshot to be saved to stable storage.
Snapshot pb.Snapshot
// CommittedEntries specifies entries to be committed to a
// store/state-machine. These have previously been committed to stable
// store.
2014-08-28 05:53:18 +04:00
CommittedEntries []pb.Entry
// Messages specifies outbound messages to be sent AFTER Entries are
// committed to stable storage.
2014-08-28 05:53:18 +04:00
Messages []pb.Message
}
func isHardStateEqual(a, b pb.HardState) bool {
2014-09-16 01:34:23 +04:00
return a.Term == b.Term && a.Vote == b.Vote && a.Commit == b.Commit
2014-08-25 07:09:06 +04:00
}
2014-05-29 00:53:26 +04:00
2014-10-08 02:22:35 +04:00
// IsEmptyHardState returns true if the given HardState is empty.
func IsEmptyHardState(st pb.HardState) bool {
return isHardStateEqual(st, emptyState)
2014-09-09 08:45:10 +04:00
}
2014-10-08 02:22:35 +04:00
// IsEmptySnap returns true if the given Snapshot is empty.
2014-09-17 05:18:45 +04:00
func IsEmptySnap(sp pb.Snapshot) bool {
return sp.Metadata.Index == 0
2014-09-17 05:18:45 +04:00
}
func (rd Ready) containsUpdates() bool {
return rd.SoftState != nil || !IsEmptyHardState(rd.HardState) ||
!IsEmptySnap(rd.Snapshot) || len(rd.Entries) > 0 ||
len(rd.CommittedEntries) > 0 || len(rd.Messages) > 0
2014-06-05 21:49:34 +04:00
}
2014-10-08 02:22:35 +04:00
// Node represents a node in a raft cluster.
2014-09-17 23:23:44 +04:00
type Node interface {
// Tick increments the internal logical clock for the Node by a single tick. Election
// timeouts and heartbeat timeouts are in units of ticks.
Tick()
// Campaign causes the Node to transition to candidate state and start campaigning to become leader.
2014-09-17 23:23:44 +04:00
Campaign(ctx context.Context) error
// Propose proposes that data be appended to the log.
Propose(ctx context.Context, data []byte) error
2014-09-23 23:02:44 +04:00
// ProposeConfChange proposes config change.
// At most one ConfChange can be in the process of going through consensus.
// Application needs to call ApplyConfChange when applying EntryConfChange type entry.
ProposeConfChange(ctx context.Context, cc pb.ConfChange) error
2014-09-17 23:23:44 +04:00
// Step advances the state machine using the given message. ctx.Err() will be returned, if any.
Step(ctx context.Context, msg pb.Message) error
// Ready returns a channel that returns the current point-in-time state
// Users of the Node must call Advance after applying the state returned by Ready
2014-09-17 23:23:44 +04:00
Ready() <-chan Ready
// Advance notifies the Node that the application has applied and saved progress up to the last Ready.
// It prepares the node to return the next available Ready.
Advance()
2014-09-23 23:02:44 +04:00
// ApplyConfChange applies config change to the local node.
// Returns an opaque ConfState protobuf which must be recorded
// in snapshots. Will never return nil; it returns a pointer only
// to match MemoryStorage.Compact.
ApplyConfChange(cc pb.ConfChange) *pb.ConfState
// Status returns the current status of the raft state machine.
Status() Status
2014-09-17 23:23:44 +04:00
// Stop performs any necessary termination of the Node
Stop()
}
type Peer struct {
ID uint64
Context []byte
}
2014-09-17 23:23:44 +04:00
// StartNode returns a new Node given a unique raft id, a list of raft peers, and
// the election and heartbeat timeouts in units of ticks.
// It appends a ConfChangeAddNode entry for each given peer to the initial log.
func StartNode(id uint64, peers []Peer, election, heartbeat int, storage Storage) Node {
2014-09-05 08:15:39 +04:00
n := newNode()
r := newRaft(id, nil, election, heartbeat, storage, 0)
// become the follower at term 1 and apply initial configuration
// entires of term 1
r.becomeFollower(1, None)
2014-10-11 15:20:14 +04:00
for _, peer := range peers {
cc := pb.ConfChange{Type: pb.ConfChangeAddNode, NodeID: peer.ID, Context: peer.Context}
2014-10-11 15:20:14 +04:00
d, err := cc.Marshal()
if err != nil {
panic("unexpected marshal error")
}
2014-10-11 15:20:14 +04:00
e := pb.Entry{Type: pb.EntryConfChange, Term: 1, Index: r.raftLog.lastIndex() + 1, Data: d}
2014-12-02 19:50:15 +03:00
r.raftLog.append(e)
}
// Mark these initial entries as committed.
// TODO(bdarnell): These entries are still unstable; do we need to preserve
// the invariant that committed < unstable?
2014-10-11 15:20:14 +04:00
r.raftLog.committed = r.raftLog.lastIndex()
r.Commit = r.raftLog.committed
// Now apply them, mainly so that the application can call Campaign
// immediately after StartNode in tests. Note that these nodes will
// be added to raft twice: here and when the application's Ready
// loop calls ApplyConfChange. The calls to addNode must come after
// all calls to raftLog.append so progress.next is set after these
// bootstrapping entries (it is an error if we try to append these
// entries since they have already been committed).
// We do not set raftLog.applied so the application will be able
// to observe all conf changes via Ready.CommittedEntries.
for _, peer := range peers {
r.addNode(peer.ID)
}
2014-09-05 08:15:39 +04:00
go n.run(r)
2014-09-17 23:23:44 +04:00
return &n
2014-09-05 08:15:39 +04:00
}
// RestartNode is similar to StartNode but does not take a list of peers.
// The current membership of the cluster will be restored from the Storage.
// If the caller has an existing state machine, pass in the last log index that
// has been applied to it; otherwise use zero.
func RestartNode(id uint64, election, heartbeat int, storage Storage, applied uint64) Node {
2014-09-05 08:15:39 +04:00
n := newNode()
r := newRaft(id, nil, election, heartbeat, storage, applied)
2014-09-05 08:15:39 +04:00
go n.run(r)
2014-09-17 23:23:44 +04:00
return &n
}
// node is the canonical implementation of the Node interface
type node struct {
propc chan pb.Message
recvc chan pb.Message
confc chan pb.ConfChange
confstatec chan pb.ConfState
readyc chan Ready
advancec chan struct{}
tickc chan struct{}
done chan struct{}
stop chan struct{}
2015-01-19 02:23:50 +03:00
status chan chan Status
2014-09-05 08:15:39 +04:00
}
2014-09-17 23:23:44 +04:00
func newNode() node {
return node{
propc: make(chan pb.Message),
recvc: make(chan pb.Message),
confc: make(chan pb.ConfChange),
confstatec: make(chan pb.ConfState),
readyc: make(chan Ready),
advancec: make(chan struct{}),
tickc: make(chan struct{}),
done: make(chan struct{}),
stop: make(chan struct{}),
2015-01-19 02:23:50 +03:00
status: make(chan chan Status),
2014-08-25 07:09:06 +04:00
}
}
2014-09-17 23:23:44 +04:00
func (n *node) Stop() {
select {
case n.stop <- struct{}{}:
// Not already stopped, so trigger it
case <-n.done:
// Node has already been stopped - no need to do anything
return
}
// Block until the stop has been acknowledged by run()
<-n.done
2014-09-01 05:49:07 +04:00
}
2014-09-17 23:23:44 +04:00
func (n *node) run(r *raft) {
var propc chan pb.Message
var readyc chan Ready
var advancec chan struct{}
var prevLastUnstablei, prevLastUnstablet uint64
var havePrevLastUnstablei bool
var prevSnapi uint64
var rd Ready
2014-06-20 01:39:17 +04:00
2014-09-16 04:35:02 +04:00
lead := None
prevSoftSt := r.softState()
prevHardSt := r.HardState
2014-09-05 08:15:39 +04:00
2014-08-25 07:09:06 +04:00
for {
if advancec != nil {
2014-09-16 04:35:02 +04:00
readyc = nil
} else {
rd = newReady(r, prevSoftSt, prevHardSt)
if rd.containsUpdates() {
readyc = n.readyc
} else {
readyc = nil
}
}
2014-09-16 04:35:02 +04:00
if lead != r.lead {
if r.hasLeader() {
if lead == None {
log.Printf("raft.node: %x elected leader %x at term %d", r.id, r.lead, r.Term)
} else {
log.Printf("raft.node: %x changed leader from %x to %x at term %d", r.id, lead, r.lead, r.Term)
}
propc = n.propc
} else {
2014-12-05 07:58:02 +03:00
log.Printf("raft.node: %x lost leader %x at term %d", r.id, lead, r.Term)
propc = nil
2014-09-03 07:21:58 +04:00
}
lead = r.lead
}
2014-08-25 07:09:06 +04:00
select {
// TODO: maybe buffer the config propose if there exists one (the way
// described in raft dissertation)
// Currently it is dropped in Step silently.
case m := <-propc:
m.From = r.id
r.Step(m)
2014-08-25 07:09:06 +04:00
case m := <-n.recvc:
// filter out response message from unknow From.
if _, ok := r.prs[m.From]; ok || !IsResponseMsg(m) {
r.Step(m) // raft never returns an error
}
case cc := <-n.confc:
if cc.NodeID == None {
r.resetPendingConf()
select {
case n.confstatec <- pb.ConfState{Nodes: r.nodes()}:
case <-n.done:
}
break
}
switch cc.Type {
2014-09-23 23:02:44 +04:00
case pb.ConfChangeAddNode:
r.addNode(cc.NodeID)
2014-09-23 23:02:44 +04:00
case pb.ConfChangeRemoveNode:
// block incoming proposal when local node is
// removed
if cc.NodeID == r.id {
n.propc = nil
}
r.removeNode(cc.NodeID)
case pb.ConfChangeUpdateNode:
r.resetPendingConf()
default:
panic("unexpected conf type")
}
select {
case n.confstatec <- pb.ConfState{Nodes: r.nodes()}:
case <-n.done:
}
2014-08-25 07:09:06 +04:00
case <-n.tickc:
2014-09-03 03:59:29 +04:00
r.tick()
case readyc <- rd:
2014-09-16 04:35:02 +04:00
if rd.SoftState != nil {
prevSoftSt = rd.SoftState
}
if len(rd.Entries) > 0 {
prevLastUnstablei = rd.Entries[len(rd.Entries)-1].Index
prevLastUnstablet = rd.Entries[len(rd.Entries)-1].Term
havePrevLastUnstablei = true
}
if !IsEmptyHardState(rd.HardState) {
2014-09-16 04:35:02 +04:00
prevHardSt = rd.HardState
}
2014-09-17 05:18:45 +04:00
if !IsEmptySnap(rd.Snapshot) {
prevSnapi = rd.Snapshot.Metadata.Index
2014-09-17 05:18:45 +04:00
}
2014-08-25 07:09:06 +04:00
r.msgs = nil
advancec = n.advancec
case <-advancec:
if prevHardSt.Commit != 0 {
r.raftLog.appliedTo(prevHardSt.Commit)
}
if havePrevLastUnstablei {
r.raftLog.stableTo(prevLastUnstablei, prevLastUnstablet)
havePrevLastUnstablei = false
}
r.raftLog.stableSnapTo(prevSnapi)
advancec = nil
2015-01-19 02:23:50 +03:00
case c := <-n.status:
c <- getStatus(r)
2014-11-12 23:32:20 +03:00
case <-n.stop:
close(n.done)
2014-08-25 07:09:06 +04:00
return
2014-07-15 10:41:19 +04:00
}
2014-05-29 00:53:26 +04:00
}
}
2014-06-05 21:49:34 +04:00
// Tick increments the internal logical clock for this Node. Election timeouts
// and heartbeat timeouts are in units of ticks.
2014-09-17 23:23:44 +04:00
func (n *node) Tick() {
2014-08-25 07:09:06 +04:00
select {
case n.tickc <- struct{}{}:
2014-09-01 05:49:07 +04:00
case <-n.done:
2014-06-08 13:50:39 +04:00
}
2014-06-05 21:49:34 +04:00
}
func (n *node) Campaign(ctx context.Context) error { return n.step(ctx, pb.Message{Type: pb.MsgHup}) }
2014-08-29 03:41:42 +04:00
2014-09-17 23:23:44 +04:00
func (n *node) Propose(ctx context.Context, data []byte) error {
2014-10-12 11:34:22 +04:00
return n.step(ctx, pb.Message{Type: pb.MsgProp, Entries: []pb.Entry{{Data: data}}})
}
func (n *node) Step(ctx context.Context, m pb.Message) error {
// ignore unexpected local messages receiving over network
if IsLocalMsg(m) {
// TODO: return an error?
return nil
}
return n.step(ctx, m)
2014-07-25 01:11:53 +04:00
}
2014-07-31 04:21:27 +04:00
2014-09-23 23:02:44 +04:00
func (n *node) ProposeConfChange(ctx context.Context, cc pb.ConfChange) error {
data, err := cc.Marshal()
if err != nil {
return err
}
2014-10-12 11:34:22 +04:00
return n.Step(ctx, pb.Message{Type: pb.MsgProp, Entries: []pb.Entry{{Type: pb.EntryConfChange, Data: data}}})
}
2014-08-28 04:22:34 +04:00
// Step advances the state machine using msgs. The ctx.Err() will be returned,
2014-08-25 23:49:14 +04:00
// if any.
func (n *node) step(ctx context.Context, m pb.Message) error {
2014-08-28 04:22:34 +04:00
ch := n.recvc
2014-10-12 11:34:22 +04:00
if m.Type == pb.MsgProp {
2014-08-28 04:22:34 +04:00
ch = n.propc
}
2014-08-28 04:22:34 +04:00
select {
case ch <- m:
return nil
case <-ctx.Done():
return ctx.Err()
2014-09-01 05:49:07 +04:00
case <-n.done:
return ErrStopped
2014-08-01 02:18:44 +04:00
}
2014-07-31 04:21:27 +04:00
}
func (n *node) Ready() <-chan Ready { return n.readyc }
2014-09-09 08:50:04 +04:00
func (n *node) Advance() {
select {
case n.advancec <- struct{}{}:
case <-n.done:
}
}
func (n *node) ApplyConfChange(cc pb.ConfChange) *pb.ConfState {
var cs pb.ConfState
2014-09-17 05:18:45 +04:00
select {
case n.confc <- cc:
2014-09-17 05:18:45 +04:00
case <-n.done:
}
select {
case cs = <-n.confstatec:
case <-n.done:
}
return &cs
}
2015-01-19 02:23:50 +03:00
func (n *node) Status() Status {
c := make(chan Status)
n.status <- c
return <-c
}
func newReady(r *raft, prevSoftSt *SoftState, prevHardSt pb.HardState) Ready {
2014-09-09 08:50:04 +04:00
rd := Ready{
Entries: r.raftLog.unstableEntries(),
2014-09-09 08:50:04 +04:00
CommittedEntries: r.raftLog.nextEnts(),
Messages: r.msgs,
}
2014-09-16 04:35:02 +04:00
if softSt := r.softState(); !softSt.equal(prevSoftSt) {
rd.SoftState = softSt
}
if !isHardStateEqual(r.HardState, prevHardSt) {
2014-09-16 04:35:02 +04:00
rd.HardState = r.HardState
2014-09-09 08:50:04 +04:00
}
if r.raftLog.unstable.snapshot != nil {
rd.Snapshot = *r.raftLog.unstable.snapshot
2014-09-17 05:18:45 +04:00
}
2014-09-09 08:50:04 +04:00
return rd
}