raft: specify voters and learners via snapshot

release-3.5
Sergey Kacheev 2020-12-09 18:19:09 +07:00
parent bfc97c1123
commit ccfd00f687
No known key found for this signature in database
GPG Key ID: EA40FA2E3C32621F
8 changed files with 249 additions and 251 deletions

View File

@ -24,8 +24,8 @@ func BenchmarkOneNode(b *testing.B) {
ctx, cancel := context.WithCancel(context.Background())
defer cancel()
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
go n.run()

View File

@ -130,8 +130,8 @@ func TestNodePropose(t *testing.T) {
return nil
}
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
r := rn.raft
go n.run()
@ -173,8 +173,8 @@ func TestNodeReadIndex(t *testing.T) {
}
wrs := []ReadState{{Index: uint64(1), RequestCtx: []byte("somedata")}}
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
r := rn.raft
r.readStates = wrs
@ -215,9 +215,9 @@ func TestNodeReadIndex(t *testing.T) {
// TestDisableProposalForwarding ensures that proposals are not forwarded to
// the leader when DisableProposalForwarding is true.
func TestDisableProposalForwarding(t *testing.T) {
r1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
cfg3 := newTestConfig(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
cfg3 := newTestConfig(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
cfg3.DisableProposalForwarding = true
r3 := newRaft(cfg3)
nt := newNetwork(r1, r2, r3)
@ -247,9 +247,9 @@ func TestDisableProposalForwarding(t *testing.T) {
// TestNodeReadIndexToOldLeader ensures that raftpb.MsgReadIndex to old leader
// gets forwarded to the new leader and 'send' method does not attach its term.
func TestNodeReadIndexToOldLeader(t *testing.T) {
r1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
nt := newNetwork(r1, r2, r3)
@ -312,8 +312,8 @@ func TestNodeProposeConfig(t *testing.T) {
return nil
}
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
r := rn.raft
go n.run()
@ -351,8 +351,8 @@ func TestNodeProposeConfig(t *testing.T) {
// TestNodeProposeAddDuplicateNode ensures that two proposes to add the same node should
// not affect the later propose to add new node.
func TestNodeProposeAddDuplicateNode(t *testing.T) {
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
go n.run()
n.Campaign(context.TODO())
@ -427,7 +427,7 @@ func TestNodeProposeAddDuplicateNode(t *testing.T) {
// know who is the current leader; node will accept proposal when it knows
// who is the current leader.
func TestBlockProposal(t *testing.T) {
rn := newTestRawNode(1, []uint64{1}, 10, 1, NewMemoryStorage())
rn := newTestRawNode(1, 10, 1, newTestMemoryStorage(withPeers(1)))
n := newNode(rn)
go n.run()
defer n.Stop()
@ -467,8 +467,8 @@ func TestNodeProposeWaitDropped(t *testing.T) {
return nil
}
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
r := rn.raft
go n.run()
@ -502,8 +502,8 @@ func TestNodeProposeWaitDropped(t *testing.T) {
// TestNodeTick ensures that node.Tick() will increase the
// elapsed of the underlying raft state machine.
func TestNodeTick(t *testing.T) {
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
r := rn.raft
go n.run()
@ -523,7 +523,7 @@ func TestNodeTick(t *testing.T) {
// TestNodeStop ensures that node.Stop() blocks until the node has stopped
// processing, and that it is idempotent
func TestNodeStop(t *testing.T) {
rn := newTestRawNode(1, []uint64{1}, 10, 1, NewMemoryStorage())
rn := newTestRawNode(1, 10, 1, newTestMemoryStorage(withPeers(1)))
n := newNode(rn)
donec := make(chan struct{})
@ -813,8 +813,8 @@ func TestIsHardStateEqual(t *testing.T) {
func TestNodeProposeAddLearnerNode(t *testing.T) {
ticker := time.NewTicker(time.Millisecond * 100)
defer ticker.Stop()
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 10, 1, s)
n := newNode(rn)
go n.run()
n.Campaign(context.TODO())
@ -907,8 +907,8 @@ func TestAppendPagination(t *testing.T) {
}
func TestCommitPagination(t *testing.T) {
s := NewMemoryStorage()
cfg := newTestConfig(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
cfg := newTestConfig(1, 10, 1, s)
cfg.MaxCommittedSizePerReady = 2048
rn, err := NewRawNode(cfg)
if err != nil {
@ -973,7 +973,7 @@ func (s *ignoreSizeHintMemStorage) Entries(lo, hi uint64, maxSize uint64) ([]raf
// This wouldn't need to exploit anything about Raft-internal code paths to fail.
func TestNodeCommitPaginationAfterRestart(t *testing.T) {
s := &ignoreSizeHintMemStorage{
MemoryStorage: NewMemoryStorage(),
MemoryStorage: newTestMemoryStorage(withPeers(1)),
}
persistedHardState := raftpb.HardState{
Term: 1,
@ -996,7 +996,7 @@ func TestNodeCommitPaginationAfterRestart(t *testing.T) {
size += uint64(ent.Size())
}
cfg := newTestConfig(1, []uint64{1}, 10, 1, s)
cfg := newTestConfig(1, 10, 1, s)
// Set a MaxSizePerMsg that would suggest to Raft that the last committed entry should
// not be included in the initial rd.CommittedEntries. However, our storage will ignore
// this and *will* return it (which is how the Commit index ended up being 10 initially).

View File

@ -117,17 +117,6 @@ type Config struct {
// ID is the identity of the local raft. ID cannot be 0.
ID uint64
// peers contains the IDs of all nodes (including self) in the raft cluster. It
// should only be set when starting a new raft cluster. Restarting raft from
// previous configuration will panic if peers is set. peer is private and only
// used for testing right now.
peers []uint64
// learners contains the IDs of all learner nodes (including self if the
// local node is a learner) in the raft cluster. learners only receives
// entries from the leader node. It does not vote or promote itself.
learners []uint64
// ElectionTick is the number of Node.Tick 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
@ -330,17 +319,6 @@ func newRaft(c *Config) *raft {
panic(err) // TODO(bdarnell)
}
if len(c.peers) > 0 || len(c.learners) > 0 {
if len(cs.Voters) > 0 || len(cs.Learners) > 0 {
// TODO(bdarnell): the peers argument is always nil except in
// tests; the argument should be removed and these tests should be
// updated to specify their nodes through a snapshot.
panic("cannot specify both newRaft(peers, learners) and ConfState.(Voters, Learners)")
}
cs.Voters = c.peers
cs.Learners = c.learners
}
r := &raft{
id: c.ID,
lead: None,

View File

@ -25,7 +25,7 @@ import (
// 2. when the window is full, no more msgApp can be sent.
func TestMsgAppFlowControlFull(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
@ -61,7 +61,7 @@ func TestMsgAppFlowControlFull(t *testing.T) {
// 1. valid msgAppResp.index moves the windows to pass all smaller or equal index.
// 2. out-of-dated msgAppResp has no effect on the sliding window.
func TestMsgAppFlowControlMoveForward(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
@ -106,7 +106,7 @@ func TestMsgAppFlowControlMoveForward(t *testing.T) {
// TestMsgAppFlowControlRecvHeartbeat ensures a heartbeat response
// frees one slot if the window is full.
func TestMsgAppFlowControlRecvHeartbeat(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()

View File

@ -51,7 +51,7 @@ func TestLeaderUpdateTermFromMessage(t *testing.T) {
// it immediately reverts to follower state.
// Reference: section 5.1
func testUpdateTermFromMessage(t *testing.T, state StateType) {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
switch state {
case StateFollower:
r.becomeFollower(1, 2)
@ -82,7 +82,7 @@ func TestRejectStaleTermMessage(t *testing.T) {
called = true
return nil
}
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r.step = fakeStep
r.loadState(pb.HardState{Term: 2})
@ -96,7 +96,7 @@ func TestRejectStaleTermMessage(t *testing.T) {
// TestStartAsFollower tests that when servers start up, they begin as followers.
// Reference: section 5.2
func TestStartAsFollower(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
if r.state != StateFollower {
t.Errorf("state = %s, want %s", r.state, StateFollower)
}
@ -109,7 +109,7 @@ func TestStartAsFollower(t *testing.T) {
func TestLeaderBcastBeat(t *testing.T) {
// heartbeat interval
hi := 1
r := newTestRaft(1, []uint64{1, 2, 3}, 10, hi, NewMemoryStorage())
r := newTestRaft(1, 10, hi, newTestMemoryStorage(withPeers(1, 2, 3)))
r.becomeCandidate()
r.becomeLeader()
for i := 0; i < 10; i++ {
@ -151,7 +151,7 @@ func TestCandidateStartNewElection(t *testing.T) {
func testNonleaderStartElection(t *testing.T, state StateType) {
// election timeout
et := 10
r := newTestRaft(1, []uint64{1, 2, 3}, et, 1, NewMemoryStorage())
r := newTestRaft(1, et, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
switch state {
case StateFollower:
r.becomeFollower(1, 2)
@ -215,7 +215,7 @@ func TestLeaderElectionInOneRoundRPC(t *testing.T) {
{5, map[uint64]bool{}, StateCandidate},
}
for i, tt := range tests {
r := newTestRaft(1, idsBySize(tt.size), 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(idsBySize(tt.size)...)))
r.Step(pb.Message{From: 1, To: 1, Type: pb.MsgHup})
for id, vote := range tt.votes {
@ -248,7 +248,7 @@ func TestFollowerVote(t *testing.T) {
{2, 1, true},
}
for i, tt := range tests {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r.loadState(pb.HardState{Term: 1, Vote: tt.vote})
r.Step(pb.Message{From: tt.nvote, To: 1, Term: 1, Type: pb.MsgVote})
@ -274,7 +274,7 @@ func TestCandidateFallback(t *testing.T) {
{From: 2, To: 1, Term: 2, Type: pb.MsgApp},
}
for i, tt := range tests {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r.Step(pb.Message{From: 1, To: 1, Type: pb.MsgHup})
if r.state != StateCandidate {
t.Fatalf("unexpected state = %s, want %s", r.state, StateCandidate)
@ -307,7 +307,7 @@ func TestCandidateElectionTimeoutRandomized(t *testing.T) {
// Reference: section 5.2
func testNonleaderElectionTimeoutRandomized(t *testing.T, state StateType) {
et := 10
r := newTestRaft(1, []uint64{1, 2, 3}, et, 1, NewMemoryStorage())
r := newTestRaft(1, et, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
timeouts := make(map[int]bool)
for round := 0; round < 50*et; round++ {
switch state {
@ -353,7 +353,7 @@ func testNonleadersElectionTimeoutNonconflict(t *testing.T, state StateType) {
rs := make([]*raft, size)
ids := idsBySize(size)
for k := range rs {
rs[k] = newTestRaft(ids[k], ids, et, 1, NewMemoryStorage())
rs[k] = newTestRaft(ids[k], et, 1, newTestMemoryStorage(withPeers(ids...)))
}
conflicts := 0
for round := 0; round < 1000; round++ {
@ -395,8 +395,8 @@ func testNonleadersElectionTimeoutNonconflict(t *testing.T, state StateType) {
// Also, it writes the new entry into stable storage.
// Reference: section 5.3
func TestLeaderStartReplication(t *testing.T) {
s := NewMemoryStorage()
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1, 2, 3))
r := newTestRaft(1, 10, 1, s)
r.becomeCandidate()
r.becomeLeader()
commitNoopEntry(r, s)
@ -434,8 +434,8 @@ func TestLeaderStartReplication(t *testing.T) {
// servers eventually find out.
// Reference: section 5.3
func TestLeaderCommitEntry(t *testing.T) {
s := NewMemoryStorage()
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1, 2, 3))
r := newTestRaft(1, 10, 1, s)
r.becomeCandidate()
r.becomeLeader()
commitNoopEntry(r, s)
@ -488,8 +488,8 @@ func TestLeaderAcknowledgeCommit(t *testing.T) {
{5, map[uint64]bool{2: true, 3: true, 4: true, 5: true}, true},
}
for i, tt := range tests {
s := NewMemoryStorage()
r := newTestRaft(1, idsBySize(tt.size), 10, 1, s)
s := newTestMemoryStorage(withPeers(idsBySize(tt.size)...))
r := newTestRaft(1, 10, 1, s)
r.becomeCandidate()
r.becomeLeader()
commitNoopEntry(r, s)
@ -521,9 +521,9 @@ func TestLeaderCommitPrecedingEntries(t *testing.T) {
{{Term: 1, Index: 1}},
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2, 3))
storage.Append(tt)
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, storage)
r := newTestRaft(1, 10, 1, storage)
r.loadState(pb.HardState{Term: 2})
r.becomeCandidate()
r.becomeLeader()
@ -578,7 +578,7 @@ func TestFollowerCommitEntry(t *testing.T) {
},
}
for i, tt := range tests {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r.becomeFollower(1, 2)
r.Step(pb.Message{From: 2, To: 1, Type: pb.MsgApp, Term: 1, Entries: tt.ents, Commit: tt.commit})
@ -619,9 +619,9 @@ func TestFollowerCheckMsgApp(t *testing.T) {
{ents[1].Term + 1, ents[1].Index + 1, ents[1].Index + 1, true, 2},
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2, 3))
storage.Append(ents)
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, storage)
r := newTestRaft(1, 10, 1, storage)
r.loadState(pb.HardState{Commit: 1})
r.becomeFollower(2, 2)
@ -675,9 +675,9 @@ func TestFollowerAppendEntries(t *testing.T) {
},
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2, 3))
storage.Append([]pb.Entry{{Term: 1, Index: 1}, {Term: 2, Index: 2}})
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, storage)
r := newTestRaft(1, 10, 1, storage)
r.becomeFollower(2, 2)
r.Step(pb.Message{From: 2, To: 1, Type: pb.MsgApp, Term: 2, LogTerm: tt.term, Index: tt.index, Entries: tt.ents})
@ -744,13 +744,13 @@ func TestLeaderSyncFollowerLog(t *testing.T) {
},
}
for i, tt := range tests {
leadStorage := NewMemoryStorage()
leadStorage := newTestMemoryStorage(withPeers(1, 2, 3))
leadStorage.Append(ents)
lead := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, leadStorage)
lead := newTestRaft(1, 10, 1, leadStorage)
lead.loadState(pb.HardState{Commit: lead.raftLog.lastIndex(), Term: term})
followerStorage := NewMemoryStorage()
followerStorage := newTestMemoryStorage(withPeers(1, 2, 3))
followerStorage.Append(tt)
follower := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, followerStorage)
follower := newTestRaft(2, 10, 1, followerStorage)
follower.loadState(pb.HardState{Term: term - 1})
// It is necessary to have a three-node cluster.
// The second may have more up-to-date log than the first one, so the
@ -781,7 +781,7 @@ func TestVoteRequest(t *testing.T) {
{[]pb.Entry{{Term: 1, Index: 1}, {Term: 2, Index: 2}}, 3},
}
for j, tt := range tests {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r.Step(pb.Message{
From: 2, To: 1, Type: pb.MsgApp, Term: tt.wterm - 1, LogTerm: 0, Index: 0, Entries: tt.ents,
})
@ -842,9 +842,9 @@ func TestVoter(t *testing.T) {
{[]pb.Entry{{Term: 2, Index: 1}, {Term: 1, Index: 2}}, 1, 1, true},
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2))
storage.Append(tt.ents)
r := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
r := newTestRaft(1, 10, 1, storage)
r.Step(pb.Message{From: 2, To: 1, Type: pb.MsgVote, Term: 3, LogTerm: tt.logterm, Index: tt.index})
@ -878,9 +878,9 @@ func TestLeaderOnlyCommitsLogFromCurrentTerm(t *testing.T) {
{3, 3},
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2))
storage.Append(ents)
r := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
r := newTestRaft(1, 10, 1, storage)
r.loadState(pb.HardState{Term: 2})
// become leader at term 3
r.becomeCandidate()

View File

@ -31,8 +31,8 @@ var (
)
func TestSendingSnapshotSetPendingSnapshot(t *testing.T) {
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(testingSnap)
sm.becomeCandidate()
@ -49,8 +49,8 @@ func TestSendingSnapshotSetPendingSnapshot(t *testing.T) {
}
func TestPendingSnapshotPauseReplication(t *testing.T) {
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(testingSnap)
sm.becomeCandidate()
@ -66,8 +66,8 @@ func TestPendingSnapshotPauseReplication(t *testing.T) {
}
func TestSnapshotFailure(t *testing.T) {
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(testingSnap)
sm.becomeCandidate()
@ -89,8 +89,8 @@ func TestSnapshotFailure(t *testing.T) {
}
func TestSnapshotSucceed(t *testing.T) {
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(testingSnap)
sm.becomeCandidate()
@ -112,8 +112,8 @@ func TestSnapshotSucceed(t *testing.T) {
}
func TestSnapshotAbort(t *testing.T) {
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(testingSnap)
sm.becomeCandidate()

View File

@ -57,7 +57,7 @@ func (r *raft) readMessages() []pb.Message {
}
func TestProgressLeader(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
r.prs.Progress[2].BecomeReplicate()
@ -76,7 +76,7 @@ func TestProgressLeader(t *testing.T) {
// TestProgressResumeByHeartbeatResp ensures raft.heartbeat reset progress.paused by heartbeat response.
func TestProgressResumeByHeartbeatResp(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
@ -95,7 +95,7 @@ func TestProgressResumeByHeartbeatResp(t *testing.T) {
}
func TestProgressPaused(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
r.Step(pb.Message{From: 1, To: 1, Type: pb.MsgProp, Entries: []pb.Entry{{Data: []byte("somedata")}}})
@ -109,7 +109,7 @@ func TestProgressPaused(t *testing.T) {
}
func TestProgressFlowControl(t *testing.T) {
cfg := newTestConfig(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
cfg := newTestConfig(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
cfg.MaxInflightMsgs = 3
cfg.MaxSizePerMsg = 2048
r := newRaft(cfg)
@ -190,7 +190,7 @@ func TestUncommittedEntryLimit(t *testing.T) {
t.Fatal("entry with no Data must have zero payload size")
}
cfg := newTestConfig(1, []uint64{1, 2, 3}, 5, 1, NewMemoryStorage())
cfg := newTestConfig(1, 5, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
cfg.MaxUncommittedEntriesSize = uint64(maxEntrySize)
cfg.MaxInflightMsgs = 2 * 1024 // avoid interference
r := newRaft(cfg)
@ -322,8 +322,8 @@ func testLeaderElection(t *testing.T, preVote bool) {
// TestLearnerElectionTimeout verfies that the leader should not start election even
// when times out.
func TestLearnerElectionTimeout(t *testing.T) {
n1 := newTestLearnerRaft(1, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n2 := newTestLearnerRaft(2, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n1 := newTestLearnerRaft(1, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n2 := newTestLearnerRaft(2, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n1.becomeFollower(1, None)
n2.becomeFollower(1, None)
@ -342,8 +342,8 @@ func TestLearnerElectionTimeout(t *testing.T) {
// TestLearnerPromotion verifies that the learner should not election until
// it is promoted to a normal peer.
func TestLearnerPromotion(t *testing.T) {
n1 := newTestLearnerRaft(1, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n2 := newTestLearnerRaft(2, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n1 := newTestLearnerRaft(1, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n2 := newTestLearnerRaft(2, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n1.becomeFollower(1, None)
n2.becomeFollower(1, None)
@ -394,7 +394,7 @@ func TestLearnerPromotion(t *testing.T) {
// TestLearnerCanVote checks that a learner can vote when it receives a valid Vote request.
// See (*raft).Step for why this is necessary and correct behavior.
func TestLearnerCanVote(t *testing.T) {
n2 := newTestLearnerRaft(2, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n2 := newTestLearnerRaft(2, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n2.becomeFollower(1, None)
@ -530,7 +530,7 @@ func TestPreVoteFromAnyState(t *testing.T) {
func testVoteFromAnyState(t *testing.T, vt pb.MessageType) {
for st := StateType(0); st < numStates; st++ {
r := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
r.Term = 1
switch st {
@ -663,8 +663,8 @@ func TestLogReplication(t *testing.T) {
// TestLearnerLogReplication tests that a learner can receive entries from the leader.
func TestLearnerLogReplication(t *testing.T) {
n1 := newTestLearnerRaft(1, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n2 := newTestLearnerRaft(2, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
n1 := newTestLearnerRaft(1, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n2 := newTestLearnerRaft(2, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
nt := newNetwork(n1, n2)
@ -792,9 +792,9 @@ func TestCommitWithoutNewTermEntry(t *testing.T) {
}
func TestDuelingCandidates(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
nt := newNetwork(a, b, c)
nt.cut(1, 3)
@ -857,9 +857,9 @@ func TestDuelingCandidates(t *testing.T) {
}
func TestDuelingPreCandidates(t *testing.T) {
cfgA := newTestConfig(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
cfgB := newTestConfig(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
cfgC := newTestConfig(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
cfgA := newTestConfig(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
cfgB := newTestConfig(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
cfgC := newTestConfig(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
cfgA.PreVote = true
cfgB.PreVote = true
cfgC.PreVote = true
@ -1153,11 +1153,11 @@ func TestCommit(t *testing.T) {
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1))
storage.Append(tt.logs)
storage.hardState = pb.HardState{Term: tt.smTerm}
sm := newTestRaft(1, []uint64{1}, 10, 2, storage)
sm := newTestRaft(1, 10, 2, storage)
for j := 0; j < len(tt.matches); j++ {
id := uint64(j) + 1
if id > 1 {
@ -1188,7 +1188,7 @@ func TestPastElectionTimeout(t *testing.T) {
}
for i, tt := range tests {
sm := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
sm.electionElapsed = tt.elapse
c := 0
for j := 0; j < 10000; j++ {
@ -1215,7 +1215,7 @@ func TestStepIgnoreOldTermMsg(t *testing.T) {
called = true
return nil
}
sm := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
sm.step = fakeStep
sm.Term = 2
sm.Step(pb.Message{Type: pb.MsgApp, Term: sm.Term - 1})
@ -1255,9 +1255,9 @@ func TestHandleMsgApp(t *testing.T) {
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1))
storage.Append([]pb.Entry{{Index: 1, Term: 1}, {Index: 2, Term: 2}})
sm := newTestRaft(1, []uint64{1}, 10, 1, storage)
sm := newTestRaft(1, 10, 1, storage)
sm.becomeFollower(2, None)
sm.handleAppendEntries(tt.m)
@ -1289,9 +1289,9 @@ func TestHandleHeartbeat(t *testing.T) {
}
for i, tt := range tests {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2))
storage.Append([]pb.Entry{{Index: 1, Term: 1}, {Index: 2, Term: 2}, {Index: 3, Term: 3}})
sm := newTestRaft(1, []uint64{1, 2}, 5, 1, storage)
sm := newTestRaft(1, 5, 1, storage)
sm.becomeFollower(2, 2)
sm.raftLog.commitTo(commit)
sm.handleHeartbeat(tt.m)
@ -1310,9 +1310,9 @@ func TestHandleHeartbeat(t *testing.T) {
// TestHandleHeartbeatResp ensures that we re-send log entries when we get a heartbeat response.
func TestHandleHeartbeatResp(t *testing.T) {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2))
storage.Append([]pb.Entry{{Index: 1, Term: 1}, {Index: 2, Term: 2}, {Index: 3, Term: 3}})
sm := newTestRaft(1, []uint64{1, 2}, 5, 1, storage)
sm := newTestRaft(1, 5, 1, storage)
sm.becomeCandidate()
sm.becomeLeader()
sm.raftLog.commitTo(sm.raftLog.lastIndex())
@ -1357,7 +1357,7 @@ func TestHandleHeartbeatResp(t *testing.T) {
// readOnly readIndexQueue and pendingReadIndex map.
// related issue: https://github.com/etcd-io/etcd/issues/7571
func TestRaftFreesReadOnlyMem(t *testing.T) {
sm := newTestRaft(1, []uint64{1, 2}, 5, 1, NewMemoryStorage())
sm := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2)))
sm.becomeCandidate()
sm.becomeLeader()
sm.raftLog.commitTo(sm.raftLog.lastIndex())
@ -1405,7 +1405,7 @@ func TestRaftFreesReadOnlyMem(t *testing.T) {
// TestMsgAppRespWaitReset verifies the resume behavior of a leader
// MsgAppResp.
func TestMsgAppRespWaitReset(t *testing.T) {
sm := newTestRaft(1, []uint64{1, 2, 3}, 5, 1, NewMemoryStorage())
sm := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
sm.becomeCandidate()
sm.becomeLeader()
@ -1515,7 +1515,7 @@ func testRecvMsgVote(t *testing.T, msgType pb.MessageType) {
}
for i, tt := range tests {
sm := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
sm.state = tt.state
switch tt.state {
case StateFollower:
@ -1596,7 +1596,7 @@ func TestStateTransition(t *testing.T) {
}
}()
sm := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
sm.state = tt.from
switch tt.to {
@ -1638,7 +1638,7 @@ func TestAllServerStepdown(t *testing.T) {
tterm := uint64(3)
for i, tt := range tests {
sm := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
switch tt.state {
case StateFollower:
sm.becomeFollower(1, None)
@ -1689,9 +1689,9 @@ func TestCandidateResetTermMsgApp(t *testing.T) {
// MsgHeartbeat or MsgApp from leader, "Step" resets the term
// with leader's and reverts back to follower.
func testCandidateResetTerm(t *testing.T, mt pb.MessageType) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
nt := newNetwork(a, b, c)
@ -1746,7 +1746,7 @@ func testCandidateResetTerm(t *testing.T, mt pb.MessageType) {
}
func TestLeaderStepdownWhenQuorumActive(t *testing.T) {
sm := newTestRaft(1, []uint64{1, 2, 3}, 5, 1, NewMemoryStorage())
sm := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
sm.checkQuorum = true
@ -1764,7 +1764,7 @@ func TestLeaderStepdownWhenQuorumActive(t *testing.T) {
}
func TestLeaderStepdownWhenQuorumLost(t *testing.T) {
sm := newTestRaft(1, []uint64{1, 2, 3}, 5, 1, NewMemoryStorage())
sm := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
sm.checkQuorum = true
@ -1781,9 +1781,9 @@ func TestLeaderStepdownWhenQuorumLost(t *testing.T) {
}
func TestLeaderSupersedingWithCheckQuorum(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
a.checkQuorum = true
b.checkQuorum = true
@ -1824,9 +1824,9 @@ func TestLeaderSupersedingWithCheckQuorum(t *testing.T) {
}
func TestLeaderElectionWithCheckQuorum(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
a.checkQuorum = true
b.checkQuorum = true
@ -1873,9 +1873,9 @@ func TestLeaderElectionWithCheckQuorum(t *testing.T) {
// can disrupt the leader even if the leader still "officially" holds the lease, The
// leader is expected to step down and adopt the candidate's term
func TestFreeStuckCandidateWithCheckQuorum(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
a.checkQuorum = true
b.checkQuorum = true
@ -1940,8 +1940,8 @@ func TestFreeStuckCandidateWithCheckQuorum(t *testing.T) {
}
func TestNonPromotableVoterWithCheckQuorum(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1)))
a.checkQuorum = true
b.checkQuorum = true
@ -1979,9 +1979,9 @@ func TestNonPromotableVoterWithCheckQuorum(t *testing.T) {
// candiate's response to late leader heartbeat forces the leader
// to step down.
func TestDisruptiveFollower(t *testing.T) {
n1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n1.checkQuorum = true
n2.checkQuorum = true
@ -2100,9 +2100,9 @@ func TestDisruptiveFollower(t *testing.T) {
// Then pre-vote phase prevents this isolated node from forcing
// current leader to step down, thus less disruptions.
func TestDisruptiveFollowerPreVote(t *testing.T) {
n1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n1.checkQuorum = true
n2.checkQuorum = true
@ -2175,9 +2175,9 @@ func TestDisruptiveFollowerPreVote(t *testing.T) {
}
func TestReadOnlyOptionSafe(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
nt := newNetwork(a, b, c)
setRandomizedElectionTimeout(b, b.electionTimeout+1)
@ -2229,8 +2229,8 @@ func TestReadOnlyOptionSafe(t *testing.T) {
}
func TestReadOnlyWithLearner(t *testing.T) {
a := newTestLearnerRaft(1, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
b := newTestLearnerRaft(2, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
a := newTestLearnerRaft(1, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
b := newTestLearnerRaft(2, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
nt := newNetwork(a, b)
setRandomizedElectionTimeout(b, b.electionTimeout+1)
@ -2280,9 +2280,9 @@ func TestReadOnlyWithLearner(t *testing.T) {
}
func TestReadOnlyOptionLease(t *testing.T) {
a := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
b := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
c := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
a := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
b := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
c := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
a.readOnly.option = ReadOnlyLeaseBased
b.readOnly.option = ReadOnlyLeaseBased
c.readOnly.option = ReadOnlyLeaseBased
@ -2351,13 +2351,13 @@ func TestReadOnlyForNewLeader(t *testing.T) {
}
peers := make([]stateMachine, 0)
for _, c := range nodeConfigs {
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1, 2, 3))
storage.Append([]pb.Entry{{Index: 1, Term: 1}, {Index: 2, Term: 1}})
storage.SetHardState(pb.HardState{Term: 1, Commit: c.committed})
if c.compactIndex != 0 {
storage.Compact(c.compactIndex)
}
cfg := newTestConfig(c.id, []uint64{1, 2, 3}, 10, 1, storage)
cfg := newTestConfig(c.id, 10, 1, storage)
cfg.Applied = c.applied
raft := newRaft(cfg)
peers = append(peers, raft)
@ -2433,7 +2433,7 @@ func TestLeaderAppResp(t *testing.T) {
for i, tt := range tests {
// sm term is 1 after it becomes the leader.
// thus the last log term must be 1 to be committed.
sm := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
sm.raftLog = &raftLog{
storage: &MemoryStorage{ents: []pb.Entry{{}, {Index: 1, Term: 0}, {Index: 2, Term: 1}}},
unstable: unstable{offset: 3},
@ -2481,7 +2481,7 @@ func TestBcastBeat(t *testing.T) {
}
storage := NewMemoryStorage()
storage.ApplySnapshot(s)
sm := newTestRaft(1, nil, 10, 1, storage)
sm := newTestRaft(1, 10, 1, storage)
sm.Term = 1
sm.becomeCandidate()
@ -2540,7 +2540,7 @@ func TestRecvMsgBeat(t *testing.T) {
}
for i, tt := range tests {
sm := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
sm.raftLog = &raftLog{storage: &MemoryStorage{ents: []pb.Entry{{}, {Index: 1, Term: 0}, {Index: 2, Term: 1}}}}
sm.Term = 1
sm.state = tt.state
@ -2583,7 +2583,7 @@ func TestLeaderIncreaseNext(t *testing.T) {
}
for i, tt := range tests {
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
sm := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
sm.raftLog.append(previousEnts...)
sm.becomeCandidate()
sm.becomeLeader()
@ -2599,7 +2599,7 @@ func TestLeaderIncreaseNext(t *testing.T) {
}
func TestSendAppendForProgressProbe(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
r.readMessages()
@ -2666,7 +2666,7 @@ func TestSendAppendForProgressProbe(t *testing.T) {
}
func TestSendAppendForProgressReplicate(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
r.readMessages()
@ -2683,7 +2683,7 @@ func TestSendAppendForProgressReplicate(t *testing.T) {
}
func TestSendAppendForProgressSnapshot(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
r.readMessages()
@ -2701,9 +2701,9 @@ func TestSendAppendForProgressSnapshot(t *testing.T) {
func TestRecvMsgUnreachable(t *testing.T) {
previousEnts := []pb.Entry{{Term: 1, Index: 1}, {Term: 1, Index: 2}, {Term: 1, Index: 3}}
s := NewMemoryStorage()
s := newTestMemoryStorage(withPeers(1, 2))
s.Append(previousEnts)
r := newTestRaft(1, []uint64{1, 2}, 10, 1, s)
r := newTestRaft(1, 10, 1, s)
r.becomeCandidate()
r.becomeLeader()
r.readMessages()
@ -2731,8 +2731,8 @@ func TestRestore(t *testing.T) {
},
}
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2))
sm := newTestRaft(1, 10, 1, storage)
if ok := sm.restore(s); !ok {
t.Fatal("restore fail, want succeed")
}
@ -2770,8 +2770,8 @@ func TestRestoreWithLearner(t *testing.T) {
},
}
storage := NewMemoryStorage()
sm := newTestLearnerRaft(3, []uint64{1, 2}, []uint64{3}, 8, 2, storage)
storage := newTestMemoryStorage(withPeers(1, 2), withLearners(3))
sm := newTestLearnerRaft(3, 8, 2, storage)
if ok := sm.restore(s); !ok {
t.Error("restore fail, want succeed")
}
@ -2823,8 +2823,8 @@ func TestRestoreVoterToLearner(t *testing.T) {
},
}
storage := NewMemoryStorage()
sm := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2, 3))
sm := newTestRaft(3, 10, 1, storage)
if sm.isLearner {
t.Errorf("%x is learner, want not", sm.id)
@ -2845,8 +2845,8 @@ func TestRestoreLearnerPromotion(t *testing.T) {
},
}
storage := NewMemoryStorage()
sm := newTestLearnerRaft(3, []uint64{1, 2}, []uint64{3}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2), withLearners(3))
sm := newTestLearnerRaft(3, 10, 1, storage)
if !sm.isLearner {
t.Errorf("%x is not learner, want yes", sm.id)
@ -2872,9 +2872,9 @@ func TestLearnerReceiveSnapshot(t *testing.T) {
},
}
store := NewMemoryStorage()
n1 := newTestLearnerRaft(1, []uint64{1}, []uint64{2}, 10, 1, store)
n2 := newTestLearnerRaft(2, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
store := newTestMemoryStorage(withPeers(1), withLearners(2))
n1 := newTestLearnerRaft(1, 10, 1, store)
n2 := newTestLearnerRaft(2, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
n1.restore(s)
ready := newReady(n1, &SoftState{}, pb.HardState{})
@ -2901,8 +2901,8 @@ func TestLearnerReceiveSnapshot(t *testing.T) {
func TestRestoreIgnoreSnapshot(t *testing.T) {
previousEnts := []pb.Entry{{Term: 1, Index: 1}, {Term: 1, Index: 2}, {Term: 1, Index: 3}}
commit := uint64(1)
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1, 2}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1, 2))
sm := newTestRaft(1, 10, 1, storage)
sm.raftLog.append(previousEnts...)
sm.raftLog.commitTo(commit)
@ -2941,8 +2941,8 @@ func TestProvideSnap(t *testing.T) {
ConfState: pb.ConfState{Voters: []uint64{1, 2}},
},
}
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(s)
sm.becomeCandidate()
@ -2971,8 +2971,8 @@ func TestIgnoreProvidingSnap(t *testing.T) {
ConfState: pb.ConfState{Voters: []uint64{1, 2}},
},
}
storage := NewMemoryStorage()
sm := newTestRaft(1, []uint64{1}, 10, 1, storage)
storage := newTestMemoryStorage(withPeers(1))
sm := newTestRaft(1, 10, 1, storage)
sm.restore(s)
sm.becomeCandidate()
@ -3001,7 +3001,7 @@ func TestRestoreFromSnapMsg(t *testing.T) {
}
m := pb.Message{Type: pb.MsgSnap, From: 1, Term: 2, Snapshot: s}
sm := newTestRaft(2, []uint64{1, 2}, 10, 1, NewMemoryStorage())
sm := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
sm.Step(m)
if sm.lead != uint64(1) {
@ -3050,7 +3050,7 @@ func TestSlowNodeRestore(t *testing.T) {
// it appends the entry to log and sets pendingConf to be true.
func TestStepConfig(t *testing.T) {
// a raft that cannot make progress
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
index := r.raftLog.lastIndex()
@ -3068,7 +3068,7 @@ func TestStepConfig(t *testing.T) {
// the proposal to noop and keep its original state.
func TestStepIgnoreConfig(t *testing.T) {
// a raft that cannot make progress
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
r.becomeCandidate()
r.becomeLeader()
r.Step(pb.Message{From: 1, To: 1, Type: pb.MsgProp, Entries: []pb.Entry{{Type: pb.EntryConfChange}}})
@ -3099,7 +3099,7 @@ func TestNewLeaderPendingConfig(t *testing.T) {
{true, 1},
}
for i, tt := range tests {
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
if tt.addEntry {
mustAppendEntry(r, pb.Entry{Type: pb.EntryNormal})
}
@ -3114,7 +3114,7 @@ func TestNewLeaderPendingConfig(t *testing.T) {
// TestAddNode tests that addNode could update nodes correctly.
func TestAddNode(t *testing.T) {
r := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
r.applyConfChange(pb.ConfChange{NodeID: 2, Type: pb.ConfChangeAddNode}.AsV2())
nodes := r.prs.VoterNodes()
wnodes := []uint64{1, 2}
@ -3125,7 +3125,7 @@ func TestAddNode(t *testing.T) {
// TestAddLearner tests that addLearner could update nodes correctly.
func TestAddLearner(t *testing.T) {
r := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
// Add new learner peer.
r.applyConfChange(pb.ConfChange{NodeID: 2, Type: pb.ConfChangeAddLearnerNode}.AsV2())
if r.isLearner {
@ -3168,7 +3168,7 @@ func TestAddLearner(t *testing.T) {
// TestAddNodeCheckQuorum tests that addNode does not trigger a leader election
// immediately when checkQuorum is set.
func TestAddNodeCheckQuorum(t *testing.T) {
r := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
r.checkQuorum = true
r.becomeCandidate()
@ -3202,7 +3202,7 @@ func TestAddNodeCheckQuorum(t *testing.T) {
// TestRemoveNode tests that removeNode could update nodes and
// and removed list correctly.
func TestRemoveNode(t *testing.T) {
r := newTestRaft(1, []uint64{1, 2}, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2)))
r.applyConfChange(pb.ConfChange{NodeID: 2, Type: pb.ConfChangeRemoveNode}.AsV2())
w := []uint64{1}
if g := r.prs.VoterNodes(); !reflect.DeepEqual(g, w) {
@ -3221,7 +3221,7 @@ func TestRemoveNode(t *testing.T) {
// TestRemoveLearner tests that removeNode could update nodes and
// and removed list correctly.
func TestRemoveLearner(t *testing.T) {
r := newTestLearnerRaft(1, []uint64{1}, []uint64{2}, 10, 1, NewMemoryStorage())
r := newTestLearnerRaft(1, 10, 1, newTestMemoryStorage(withPeers(1), withLearners(2)))
r.applyConfChange(pb.ConfChange{NodeID: 2, Type: pb.ConfChangeRemoveNode}.AsV2())
w := []uint64{1}
if g := r.prs.VoterNodes(); !reflect.DeepEqual(g, w) {
@ -3254,7 +3254,7 @@ func TestPromotable(t *testing.T) {
{[]uint64{2, 3}, false},
}
for i, tt := range tests {
r := newTestRaft(id, tt.peers, 5, 1, NewMemoryStorage())
r := newTestRaft(id, 5, 1, newTestMemoryStorage(withPeers(tt.peers...)))
if g := r.promotable(); g != tt.wp {
t.Errorf("#%d: promotable = %v, want %v", i, g, tt.wp)
}
@ -3276,7 +3276,7 @@ func TestRaftNodes(t *testing.T) {
},
}
for i, tt := range tests {
r := newTestRaft(1, tt.ids, 10, 1, NewMemoryStorage())
r := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(tt.ids...)))
if !reflect.DeepEqual(r.prs.VoterNodes(), tt.wids) {
t.Errorf("#%d: nodes = %+v, want %+v", i, r.prs.VoterNodes(), tt.wids)
}
@ -3292,7 +3292,7 @@ func TestPreCampaignWhileLeader(t *testing.T) {
}
func testCampaignWhileLeader(t *testing.T, preVote bool) {
cfg := newTestConfig(1, []uint64{1}, 5, 1, NewMemoryStorage())
cfg := newTestConfig(1, 5, 1, newTestMemoryStorage(withPeers(1)))
cfg.PreVote = preVote
r := newRaft(cfg)
if r.state != StateFollower {
@ -3318,8 +3318,8 @@ func testCampaignWhileLeader(t *testing.T, preVote bool) {
// committed when a config change reduces the quorum requirements.
func TestCommitAfterRemoveNode(t *testing.T) {
// Create a cluster with two nodes.
s := NewMemoryStorage()
r := newTestRaft(1, []uint64{1, 2}, 5, 1, s)
s := newTestMemoryStorage(withPeers(1, 2))
r := newTestRaft(1, 5, 1, s)
r.becomeCandidate()
r.becomeLeader()
@ -3761,7 +3761,7 @@ func checkLeaderTransferState(t *testing.T, r *raft, state StateType, lead uint6
// (previously, if the node also got votes, it would panic as it
// transitioned to StateLeader)
func TestTransferNonMember(t *testing.T) {
r := newTestRaft(1, []uint64{2, 3, 4}, 5, 1, NewMemoryStorage())
r := newTestRaft(1, 5, 1, newTestMemoryStorage(withPeers(2, 3, 4)))
r.Step(pb.Message{From: 2, To: 1, Type: pb.MsgTimeoutNow})
r.Step(pb.Message{From: 2, To: 1, Type: pb.MsgVoteResp})
@ -3777,9 +3777,9 @@ func TestTransferNonMember(t *testing.T) {
// Previously the cluster would come to a standstill when run with PreVote
// enabled.
func TestNodeWithSmallerTermCanCompleteElection(t *testing.T) {
n1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n1.becomeFollower(1, None)
n2.becomeFollower(1, None)
@ -3872,9 +3872,9 @@ func TestNodeWithSmallerTermCanCompleteElection(t *testing.T) {
// TestPreVoteWithSplitVote verifies that after split vote, cluster can complete
// election in next round.
func TestPreVoteWithSplitVote(t *testing.T) {
n1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n1.becomeFollower(1, None)
n2.becomeFollower(1, None)
@ -3949,9 +3949,9 @@ func TestPreVoteWithSplitVote(t *testing.T) {
// TestPreVoteWithCheckQuorum ensures that after a node become pre-candidate,
// it will checkQuorum correctly.
func TestPreVoteWithCheckQuorum(t *testing.T) {
n1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n1.becomeFollower(1, None)
n2.becomeFollower(1, None)
@ -3998,9 +3998,9 @@ func TestPreVoteWithCheckQuorum(t *testing.T) {
// TestLearnerCampaign verifies that a learner won't campaign even if it receives
// a MsgHup or MsgTimeoutNow.
func TestLearnerCampaign(t *testing.T) {
n1 := newTestRaft(1, []uint64{1}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1)))
n1.applyConfChange(pb.ConfChange{NodeID: 2, Type: pb.ConfChangeAddLearnerNode}.AsV2())
n2 := newTestRaft(2, []uint64{1}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1)))
n2.applyConfChange(pb.ConfChange{NodeID: 2, Type: pb.ConfChangeAddLearnerNode}.AsV2())
nt := newNetwork(n1, n2)
nt.send(pb.Message{From: 2, To: 2, Type: pb.MsgHup})
@ -4034,9 +4034,9 @@ func TestLearnerCampaign(t *testing.T) {
// n2 is follower with term 2
// n3 is partitioned, with term 4 and less log, state is candidate
func newPreVoteMigrationCluster(t *testing.T) *network {
n1 := newTestRaft(1, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n2 := newTestRaft(2, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n3 := newTestRaft(3, []uint64{1, 2, 3}, 10, 1, NewMemoryStorage())
n1 := newTestRaft(1, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n2 := newTestRaft(2, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n3 := newTestRaft(3, 10, 1, newTestMemoryStorage(withPeers(1, 2, 3)))
n1.becomeFollower(1, None)
n2.becomeFollower(1, None)
@ -4270,7 +4270,7 @@ func entsWithConfig(configFunc func(*Config), terms ...uint64) *raft {
for i, term := range terms {
storage.Append([]pb.Entry{{Index: uint64(i + 1), Term: term}})
}
cfg := newTestConfig(1, []uint64{}, 5, 1, storage)
cfg := newTestConfig(1, 5, 1, storage)
if configFunc != nil {
configFunc(cfg)
}
@ -4285,7 +4285,7 @@ func entsWithConfig(configFunc func(*Config), terms ...uint64) *raft {
func votedWithConfig(configFunc func(*Config), vote, term uint64) *raft {
storage := NewMemoryStorage()
storage.SetHardState(pb.HardState{Vote: vote, Term: term})
cfg := newTestConfig(1, []uint64{}, 5, 1, storage)
cfg := newTestConfig(1, 5, 1, storage)
if configFunc != nil {
configFunc(cfg)
}
@ -4326,8 +4326,8 @@ func newNetworkWithConfig(configFunc func(*Config), peers ...stateMachine) *netw
id := peerAddrs[j]
switch v := p.(type) {
case nil:
nstorage[id] = NewMemoryStorage()
cfg := newTestConfig(id, peerAddrs, 10, 1, nstorage[id])
nstorage[id] = newTestMemoryStorage(withPeers(peerAddrs...))
cfg := newTestConfig(id, 10, 1, nstorage[id])
if configFunc != nil {
configFunc(cfg)
}
@ -4463,10 +4463,9 @@ func setRandomizedElectionTimeout(r *raft, v int) {
r.randomizedElectionTimeout = v
}
func newTestConfig(id uint64, peers []uint64, election, heartbeat int, storage Storage) *Config {
func newTestConfig(id uint64, election, heartbeat int, storage Storage) *Config {
return &Config{
ID: id,
peers: peers,
ElectionTick: election,
HeartbeatTick: heartbeat,
Storage: storage,
@ -4475,20 +4474,41 @@ func newTestConfig(id uint64, peers []uint64, election, heartbeat int, storage S
}
}
func newTestRaft(id uint64, peers []uint64, election, heartbeat int, storage Storage) *raft {
return newRaft(newTestConfig(id, peers, election, heartbeat, storage))
type testMemoryStorageOptions func(*MemoryStorage)
func withPeers(peers ...uint64) testMemoryStorageOptions {
return func(ms *MemoryStorage) {
ms.snapshot.Metadata.ConfState.Voters = peers
}
}
func newTestLearnerRaft(id uint64, peers []uint64, learners []uint64, election, heartbeat int, storage Storage) *raft {
cfg := newTestConfig(id, peers, election, heartbeat, storage)
cfg.learners = learners
func withLearners(learners ...uint64) testMemoryStorageOptions {
return func(ms *MemoryStorage) {
ms.snapshot.Metadata.ConfState.Learners = learners
}
}
func newTestMemoryStorage(opts ...testMemoryStorageOptions) *MemoryStorage {
ms := NewMemoryStorage()
for _, o := range opts {
o(ms)
}
return ms
}
func newTestRaft(id uint64, election, heartbeat int, storage Storage) *raft {
return newRaft(newTestConfig(id, election, heartbeat, storage))
}
func newTestLearnerRaft(id uint64, election, heartbeat int, storage Storage) *raft {
cfg := newTestConfig(id, election, heartbeat, storage)
return newRaft(cfg)
}
// newTestRawNode sets up a RawNode with the given peers. The configuration will
// not be reflected in the Storage.
func newTestRawNode(id uint64, peers []uint64, election, heartbeat int, storage Storage) *RawNode {
cfg := newTestConfig(id, peers, election, heartbeat, storage)
func newTestRawNode(id uint64, election, heartbeat int, storage Storage) *RawNode {
cfg := newTestConfig(id, election, heartbeat, storage)
rn, err := NewRawNode(cfg)
if err != nil {
panic(err)

View File

@ -88,7 +88,7 @@ func TestRawNodeStep(t *testing.T) {
}
// Append an empty entry to make sure the non-local messages (like
// vote requests) are ignored and don't trigger assertions.
rawNode, err := NewRawNode(newTestConfig(1, nil, 10, 1, s))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, s))
if err != nil {
t.Fatal(err)
}
@ -223,8 +223,8 @@ func TestRawNodeProposeAndConfChange(t *testing.T) {
for _, tc := range testCases {
t.Run("", func(t *testing.T) {
s := NewMemoryStorage()
rawNode, err := NewRawNode(newTestConfig(1, []uint64{1}, 10, 1, s))
s := newTestMemoryStorage(withPeers(1))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, s))
if err != nil {
t.Fatal(err)
}
@ -389,8 +389,8 @@ func TestRawNodeJointAutoLeave(t *testing.T) {
exp2Cs := pb.ConfState{Voters: []uint64{1}, Learners: []uint64{2}}
t.Run("", func(t *testing.T) {
s := NewMemoryStorage()
rawNode, err := NewRawNode(newTestConfig(1, []uint64{1}, 10, 1, s))
s := newTestMemoryStorage(withPeers(1))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, s))
if err != nil {
t.Fatal(err)
}
@ -509,8 +509,8 @@ func TestRawNodeJointAutoLeave(t *testing.T) {
// TestRawNodeProposeAddDuplicateNode ensures that two proposes to add the same node should
// not affect the later propose to add new node.
func TestRawNodeProposeAddDuplicateNode(t *testing.T) {
s := NewMemoryStorage()
rawNode, err := NewRawNode(newTestConfig(1, []uint64{1}, 10, 1, s))
s := newTestMemoryStorage(withPeers(1))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, s))
if err != nil {
t.Fatal(err)
}
@ -592,8 +592,8 @@ func TestRawNodeReadIndex(t *testing.T) {
}
wrs := []ReadState{{Index: uint64(1), RequestCtx: []byte("somedata")}}
s := NewMemoryStorage()
c := newTestConfig(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
c := newTestConfig(1, 10, 1, s)
rawNode, err := NewRawNode(c)
if err != nil {
t.Fatal(err)
@ -734,7 +734,7 @@ func TestRawNodeStart(t *testing.T) {
t.Fatal(err)
}
rawNode, err := NewRawNode(newTestConfig(1, nil, 10, 1, storage))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, storage))
if err != nil {
t.Fatal(err)
}
@ -775,10 +775,10 @@ func TestRawNodeRestart(t *testing.T) {
MustSync: false,
}
storage := NewMemoryStorage()
storage := newTestMemoryStorage(withPeers(1))
storage.SetHardState(st)
storage.Append(entries)
rawNode, err := NewRawNode(newTestConfig(1, []uint64{1}, 10, 1, storage))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, storage))
if err != nil {
t.Fatal(err)
}
@ -816,7 +816,7 @@ func TestRawNodeRestartFromSnapshot(t *testing.T) {
s.SetHardState(st)
s.ApplySnapshot(snap)
s.Append(entries)
rawNode, err := NewRawNode(newTestConfig(1, nil, 10, 1, s))
rawNode, err := NewRawNode(newTestConfig(1, 10, 1, s))
if err != nil {
t.Fatal(err)
}
@ -834,8 +834,8 @@ func TestRawNodeRestartFromSnapshot(t *testing.T) {
// no dependency check between Ready() and Advance()
func TestRawNodeStatus(t *testing.T) {
s := NewMemoryStorage()
rn, err := NewRawNode(newTestConfig(1, []uint64{1}, 10, 1, s))
s := newTestMemoryStorage(withPeers(1))
rn, err := NewRawNode(newTestConfig(1, 10, 1, s))
if err != nil {
t.Fatal(err)
}
@ -881,7 +881,7 @@ func TestRawNodeStatus(t *testing.T) {
// write.
func TestRawNodeCommitPaginationAfterRestart(t *testing.T) {
s := &ignoreSizeHintMemStorage{
MemoryStorage: NewMemoryStorage(),
MemoryStorage: newTestMemoryStorage(withPeers(1)),
}
persistedHardState := pb.HardState{
Term: 1,
@ -904,7 +904,7 @@ func TestRawNodeCommitPaginationAfterRestart(t *testing.T) {
size += uint64(ent.Size())
}
cfg := newTestConfig(1, []uint64{1}, 10, 1, s)
cfg := newTestConfig(1, 10, 1, s)
// Set a MaxSizePerMsg that would suggest to Raft that the last committed entry should
// not be included in the initial rd.CommittedEntries. However, our storage will ignore
// this and *will* return it (which is how the Commit index ended up being 10 initially).
@ -953,8 +953,8 @@ func TestRawNodeBoundedLogGrowthWithPartition(t *testing.T) {
testEntry := pb.Entry{Data: data}
maxEntrySize := uint64(maxEntries * PayloadSize(testEntry))
s := NewMemoryStorage()
cfg := newTestConfig(1, []uint64{1}, 10, 1, s)
s := newTestMemoryStorage(withPeers(1))
cfg := newTestConfig(1, 10, 1, s)
cfg.MaxUncommittedEntriesSize = maxEntrySize
rawNode, err := NewRawNode(cfg)
if err != nil {
@ -1010,7 +1010,7 @@ func BenchmarkStatus(b *testing.B) {
for i := range peers {
peers[i] = uint64(i + 1)
}
cfg := newTestConfig(1, peers, 3, 1, NewMemoryStorage())
cfg := newTestConfig(1, 3, 1, newTestMemoryStorage(withPeers(peers...)))
cfg.Logger = discardLogger
r := newRaft(cfg)
r.becomeFollower(1, 1)
@ -1075,8 +1075,8 @@ func BenchmarkStatus(b *testing.B) {
func TestRawNodeConsumeReady(t *testing.T) {
// Check that readyWithoutAccept() does not call acceptReady (which resets
// the messages) but Ready() does.
s := NewMemoryStorage()
rn := newTestRawNode(1, []uint64{1}, 3, 1, s)
s := newTestMemoryStorage(withPeers(1))
rn := newTestRawNode(1, 3, 1, s)
m1 := pb.Message{Context: []byte("foo")}
m2 := pb.Message{Context: []byte("bar")}