etcd/raft/rafttest/node.go

146 lines
3.2 KiB
Go
Raw Normal View History

2016-05-13 06:49:15 +03:00
// Copyright 2015 The etcd Authors
2015-09-15 23:54:11 +03:00
//
// 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.
2015-02-01 07:51:50 +03:00
package rafttest
import (
"log"
"time"
"github.com/coreos/etcd/raft"
"github.com/coreos/etcd/raft/raftpb"
2016-03-23 03:10:28 +03:00
"golang.org/x/net/context"
2015-02-01 07:51:50 +03:00
)
type node struct {
raft.Node
id uint64
iface iface
2015-02-01 07:51:50 +03:00
stopc chan struct{}
2015-02-10 03:26:43 +03:00
pausec chan bool
2015-02-01 07:51:50 +03:00
// stable
storage *raft.MemoryStorage
state raftpb.HardState
}
func startNode(id uint64, peers []raft.Peer, iface iface) *node {
2015-02-01 07:51:50 +03:00
st := raft.NewMemoryStorage()
2015-03-28 21:46:22 +03:00
c := &raft.Config{
ID: id,
ElectionTick: 10,
HeartbeatTick: 1,
Storage: st,
MaxSizePerMsg: 1024 * 1024,
MaxInflightMsgs: 256,
}
rn := raft.StartNode(c, peers)
2015-02-01 07:51:50 +03:00
n := &node{
Node: rn,
id: id,
2015-02-01 07:51:50 +03:00
storage: st,
iface: iface,
2015-02-10 03:26:43 +03:00
pausec: make(chan bool),
2015-02-01 07:51:50 +03:00
}
n.start()
return n
}
2015-02-01 07:51:50 +03:00
func (n *node) start() {
n.stopc = make(chan struct{})
2015-02-01 07:51:50 +03:00
ticker := time.Tick(5 * time.Millisecond)
2015-02-01 07:51:50 +03:00
go func() {
for {
select {
case <-ticker:
n.Tick()
case rd := <-n.Ready():
if !raft.IsEmptyHardState(rd.HardState) {
n.state = rd.HardState
n.storage.SetHardState(n.state)
2015-02-01 07:51:50 +03:00
}
n.storage.Append(rd.Entries)
2015-03-28 21:46:22 +03:00
time.Sleep(time.Millisecond)
2015-02-06 21:03:52 +03:00
// TODO: make send async, more like real world...
for _, m := range rd.Messages {
n.iface.send(m)
}
2015-02-01 07:51:50 +03:00
n.Advance()
case m := <-n.iface.recv():
2015-02-01 07:51:50 +03:00
n.Step(context.TODO(), m)
case <-n.stopc:
n.Stop()
log.Printf("raft.%d: stop", n.id)
n.Node = nil
close(n.stopc)
2015-02-01 07:51:50 +03:00
return
2015-02-10 03:26:43 +03:00
case p := <-n.pausec:
recvms := make([]raftpb.Message, 0)
for p {
select {
case m := <-n.iface.recv():
recvms = append(recvms, m)
case p = <-n.pausec:
}
}
// step all pending messages
for _, m := range recvms {
n.Step(context.TODO(), m)
}
2015-02-01 07:51:50 +03:00
}
}
}()
}
// stop stops the node. stop a stopped node might panic.
// All in memory state of node is discarded.
2015-02-01 07:51:50 +03:00
// All stable MUST be unchanged.
func (n *node) stop() {
n.iface.disconnect()
n.stopc <- struct{}{}
// wait for the shutdown
<-n.stopc
}
// restart restarts the node. restart a started node
// blocks and might affect the future stop operation.
func (n *node) restart() {
// wait for the shutdown
<-n.stopc
2015-03-28 21:46:22 +03:00
c := &raft.Config{
ID: n.id,
ElectionTick: 10,
HeartbeatTick: 1,
Storage: n.storage,
MaxSizePerMsg: 1024 * 1024,
MaxInflightMsgs: 256,
}
n.Node = raft.RestartNode(c)
n.start()
n.iface.connect()
2015-02-01 07:51:50 +03:00
}
// pause pauses the node.
// The paused node buffers the received messages and replies
// all of them when it resumes.
func (n *node) pause() {
2015-02-10 03:26:43 +03:00
n.pausec <- true
2015-02-01 07:51:50 +03:00
}
// resume resumes the paused node.
func (n *node) resume() {
2015-02-10 03:26:43 +03:00
n.pausec <- false
2015-02-01 07:51:50 +03:00
}