diff --git a/interaction_test.go b/interaction_test.go index 31db89be0918..56d18200fc1e 100644 --- a/interaction_test.go +++ b/interaction_test.go @@ -19,6 +19,7 @@ import ( "github.com/cockroachdb/datadriven" + "go.etcd.io/raft/v3" "go.etcd.io/raft/v3/rafttest" ) @@ -27,7 +28,9 @@ func TestInteraction(t *testing.T) { // diff. Only commit the changes if you understand what caused them and if // they are desired. datadriven.Walk(t, "testdata", func(t *testing.T, path string) { - env := rafttest.NewInteractionEnv(nil) + env := rafttest.NewInteractionEnv(&rafttest.InteractionOpts{ + SetRandomizedElectionTimeout: raft.SetRandomizedElectionTimeout, + }) datadriven.RunTest(t, path, func(t *testing.T, d *datadriven.TestData) string { return env.Handle(t, *d) }) diff --git a/raft_test.go b/raft_test.go index 64337d37a5d8..69bd8fa48813 100644 --- a/raft_test.go +++ b/raft_test.go @@ -5000,6 +5000,12 @@ func setRandomizedElectionTimeout(r *raft, v int) { r.randomizedElectionTimeout = v } +// SetRandomizedElectionTimeout is like setRandomizedElectionTimeout, but +// exported for use by tests that are not in the raft package, using RawNode. +func SetRandomizedElectionTimeout(r *RawNode, v int) { + setRandomizedElectionTimeout(r.raft, v) +} + func newTestConfig(id uint64, election, heartbeat int, storage Storage) *Config { return &Config{ ID: id, diff --git a/rafttest/interaction_env.go b/rafttest/interaction_env.go index a7dfc0cf6caf..16881c45fdc3 100644 --- a/rafttest/interaction_env.go +++ b/rafttest/interaction_env.go @@ -27,6 +27,10 @@ import ( // InteractionOpts groups the options for an InteractionEnv. type InteractionOpts struct { OnConfig func(*raft.Config) + + // SetRandomizedElectionTimeout is used to plumb this function down from the + // raft test package. + SetRandomizedElectionTimeout func(node *raft.RawNode, timeout int) } // Node is a member of a raft group tested via an InteractionEnv. diff --git a/rafttest/interaction_env_handler.go b/rafttest/interaction_env_handler.go index 74845fd1d6ab..d78adbb70807 100644 --- a/rafttest/interaction_env_handler.go +++ b/rafttest/interaction_env_handler.go @@ -97,6 +97,14 @@ func (env *InteractionEnv) Handle(t *testing.T, d datadriven.TestData) string { // following, etc.). The information for node n is based on // n's view. err = env.handleRaftState() + case "set-randomized-election-timeout": + // Set the randomized election timeout for the given node. Will be reset + // again when the node changes state. + // + // Example: + // + // set-randomized-election-timeout 1 timeout=5 + err = env.handleSetRandomizedElectionTimeout(t, d) case "stabilize": // Deliver messages to and run process-ready on the set of IDs until // no more work is to be done. If no ids are given, all nodes are used. @@ -112,6 +120,14 @@ func (env *InteractionEnv) Handle(t *testing.T, d datadriven.TestData) string { // // status 5 err = env.handleStatus(t, d) + case "tick-election": + // Tick an election timeout interval for the given node (but beware the + // randomized timeout). + // + // Example: + // + // tick-election 3 + err = env.handleTickElection(t, d) case "tick-heartbeat": // Tick a heartbeat interval. // diff --git a/rafttest/interaction_env_handler_add_nodes.go b/rafttest/interaction_env_handler_add_nodes.go index f086aed6e7c7..3f2dd629a349 100644 --- a/rafttest/interaction_env_handler_add_nodes.go +++ b/rafttest/interaction_env_handler_add_nodes.go @@ -50,6 +50,10 @@ func (env *InteractionEnv) handleAddNodes(t *testing.T, d datadriven.TestData) e arg.Scan(t, i, &snap.Data) case "async-storage-writes": arg.Scan(t, i, &cfg.AsyncStorageWrites) + case "prevote": + arg.Scan(t, i, &cfg.PreVote) + case "checkquorum": + arg.Scan(t, i, &cfg.CheckQuorum) } } } diff --git a/rafttest/interaction_env_handler_set_randomized_election_timeout.go b/rafttest/interaction_env_handler_set_randomized_election_timeout.go new file mode 100644 index 000000000000..36f0cd2d8d9b --- /dev/null +++ b/rafttest/interaction_env_handler_set_randomized_election_timeout.go @@ -0,0 +1,34 @@ +// Copyright 2023 The etcd Authors +// +// 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 rafttest + +import ( + "testing" + + "github.com/cockroachdb/datadriven" + "github.com/stretchr/testify/require" +) + +func (env *InteractionEnv) handleSetRandomizedElectionTimeout( + t *testing.T, d datadriven.TestData, +) error { + idx := firstAsNodeIdx(t, d) + var timeout int + d.ScanArgs(t, "timeout", &timeout) + require.NotZero(t, timeout) + + env.Options.SetRandomizedElectionTimeout(env.Nodes[idx].RawNode, timeout) + return nil +} diff --git a/rafttest/interaction_env_handler_tick_heartbeat.go b/rafttest/interaction_env_handler_tick.go similarity index 85% rename from rafttest/interaction_env_handler_tick_heartbeat.go rename to rafttest/interaction_env_handler_tick.go index 349ca78efad2..10c9ba2130ef 100644 --- a/rafttest/interaction_env_handler_tick_heartbeat.go +++ b/rafttest/interaction_env_handler_tick.go @@ -20,6 +20,11 @@ import ( "github.com/cockroachdb/datadriven" ) +func (env *InteractionEnv) handleTickElection(t *testing.T, d datadriven.TestData) error { + idx := firstAsNodeIdx(t, d) + return env.Tick(idx, env.Nodes[idx].Config.ElectionTick) +} + func (env *InteractionEnv) handleTickHeartbeat(t *testing.T, d datadriven.TestData) error { idx := firstAsNodeIdx(t, d) return env.Tick(idx, env.Nodes[idx].Config.HeartbeatTick) diff --git a/testdata/checkquorum.txt b/testdata/checkquorum.txt new file mode 100644 index 000000000000..ed69a529dfed --- /dev/null +++ b/testdata/checkquorum.txt @@ -0,0 +1,236 @@ +# Tests that CheckQuorum causes a leader to step down if it hasn't heard from a +# quorum of followers in the past election timeout interval. +# +# Also tests that votes are rejected when there is a current leader. In the Raft +# thesis this is part of PreVote, but etcd/raft enables this via CheckQuorum. + +log-level none +---- +ok + +add-nodes 3 voters=(1,2,3) index=10 checkquorum=true +---- +ok + +campaign 1 +---- +ok + +stabilize +---- +ok (quiet) + +log-level debug +---- +ok + +# Campaigning will fail when there is an active leader. +campaign 2 +---- +INFO 2 is starting a new election at term 1 +INFO 2 became candidate at term 2 +INFO 2 [logterm: 1, index: 11] sent MsgVote request to 1 at term 2 +INFO 2 [logterm: 1, index: 11] sent MsgVote request to 3 at term 2 + +stabilize +---- +> 2 handling Ready + Ready MustSync=true: + Lead:0 State:StateCandidate + HardState Term:2 Vote:2 Commit:11 + Messages: + 2->1 MsgVote Term:2 Log:1/11 + 2->3 MsgVote Term:2 Log:1/11 + INFO 2 received MsgVoteResp from 2 at term 2 + INFO 2 has received 1 MsgVoteResp votes and 0 vote rejections +> 1 receiving messages + 2->1 MsgVote Term:2 Log:1/11 + INFO 1 [logterm: 1, index: 11, vote: 1] ignored MsgVote from 2 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) +> 3 receiving messages + 2->3 MsgVote Term:2 Log:1/11 + INFO 3 [logterm: 1, index: 11, vote: 1] ignored MsgVote from 2 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) + +# Tick the leader without processing any messages from followers. We have to +# tick 2 election timeouts, since the followers were active in the current +# interval (see messages above). +tick-election 1 +---- +ok + +tick-election 1 +---- +WARN 1 stepped down to follower since quorum is not active +INFO 1 became follower at term 1 + +# We'll now send all of the heartbeats that were buffered during the ticks +# above. Conceptually, "the network was slow". +stabilize +---- +> 1 handling Ready + Ready MustSync=false: + Lead:0 State:StateFollower + Messages: + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 +> 2 receiving messages + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->2 MsgHeartbeat Term:1 Log:0/0 Commit:11 +> 3 receiving messages + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 + 1->3 MsgHeartbeat Term:1 Log:0/0 Commit:11 +> 2 handling Ready + Ready MustSync=false: + Messages: + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 +> 3 handling Ready + Ready MustSync=false: + Messages: + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + 3->1 MsgHeartbeatResp Term:1 Log:0/0 +> 1 receiving messages + 2->1 MsgAppResp Term:2 Log:0/0 + INFO 1 [term: 1] received a MsgAppResp message with higher term from 2 [term: 2] + INFO 1 became follower at term 2 + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 + 2->1 MsgAppResp Term:2 Log:0/0 + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + INFO 1 [term: 2] ignored a MsgHeartbeatResp message with lower term from 3 [term: 1] + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + INFO 1 [term: 2] ignored a MsgHeartbeatResp message with lower term from 3 [term: 1] + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + INFO 1 [term: 2] ignored a MsgHeartbeatResp message with lower term from 3 [term: 1] + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + INFO 1 [term: 2] ignored a MsgHeartbeatResp message with lower term from 3 [term: 1] + 3->1 MsgHeartbeatResp Term:1 Log:0/0 + INFO 1 [term: 2] ignored a MsgHeartbeatResp message with lower term from 3 [term: 1] +> 1 handling Ready + Ready MustSync=true: + HardState Term:2 Commit:11 + +# Other nodes can now successfully campaign. Note that we haven't ticked 3, so +# it won't grant votes. +campaign 2 +---- +INFO 2 is starting a new election at term 2 +INFO 2 became candidate at term 3 +INFO 2 [logterm: 1, index: 11] sent MsgVote request to 1 at term 3 +INFO 2 [logterm: 1, index: 11] sent MsgVote request to 3 at term 3 + +process-ready 2 +---- +Ready MustSync=true: +HardState Term:3 Vote:2 Commit:11 +Messages: +2->1 MsgVote Term:3 Log:1/11 +2->3 MsgVote Term:3 Log:1/11 +INFO 2 received MsgVoteResp from 2 at term 3 +INFO 2 has received 1 MsgVoteResp votes and 0 vote rejections + +deliver-msgs 1 +---- +2->1 MsgVote Term:3 Log:1/11 +INFO 1 [term: 2] received a MsgVote message with higher term from 2 [term: 3] +INFO 1 became follower at term 3 +INFO 1 [logterm: 1, index: 11, vote: 0] cast MsgVote for 2 [logterm: 1, index: 11] at term 3 + +deliver-msgs 3 +---- +2->3 MsgVote Term:3 Log:1/11 +INFO 3 [logterm: 1, index: 11, vote: 1] ignored MsgVote from 2 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) + +stabilize +---- +> 1 handling Ready + Ready MustSync=true: + HardState Term:3 Vote:2 Commit:11 + Messages: + 1->2 MsgVoteResp Term:3 Log:0/0 +> 2 receiving messages + 1->2 MsgVoteResp Term:3 Log:0/0 + INFO 2 received MsgVoteResp from 1 at term 3 + INFO 2 has received 2 MsgVoteResp votes and 0 vote rejections + INFO 2 became leader at term 3 +> 2 handling Ready + Ready MustSync=true: + Lead:2 State:StateLeader + Entries: + 3/12 EntryNormal "" + Messages: + 2->1 MsgApp Term:3 Log:1/11 Commit:11 Entries:[3/12 EntryNormal ""] + 2->3 MsgApp Term:3 Log:1/11 Commit:11 Entries:[3/12 EntryNormal ""] +> 1 receiving messages + 2->1 MsgApp Term:3 Log:1/11 Commit:11 Entries:[3/12 EntryNormal ""] +> 3 receiving messages + 2->3 MsgApp Term:3 Log:1/11 Commit:11 Entries:[3/12 EntryNormal ""] + INFO 3 [term: 1] received a MsgApp message with higher term from 2 [term: 3] + INFO 3 became follower at term 3 +> 1 handling Ready + Ready MustSync=true: + Lead:2 State:StateFollower + Entries: + 3/12 EntryNormal "" + Messages: + 1->2 MsgAppResp Term:3 Log:0/12 +> 3 handling Ready + Ready MustSync=true: + Lead:2 State:StateFollower + HardState Term:3 Commit:11 + Entries: + 3/12 EntryNormal "" + Messages: + 3->2 MsgAppResp Term:3 Log:0/12 +> 2 receiving messages + 1->2 MsgAppResp Term:3 Log:0/12 + 3->2 MsgAppResp Term:3 Log:0/12 +> 2 handling Ready + Ready MustSync=false: + HardState Term:3 Vote:2 Commit:12 + CommittedEntries: + 3/12 EntryNormal "" + Messages: + 2->1 MsgApp Term:3 Log:3/12 Commit:12 + 2->3 MsgApp Term:3 Log:3/12 Commit:12 +> 1 receiving messages + 2->1 MsgApp Term:3 Log:3/12 Commit:12 +> 3 receiving messages + 2->3 MsgApp Term:3 Log:3/12 Commit:12 +> 1 handling Ready + Ready MustSync=false: + HardState Term:3 Vote:2 Commit:12 + CommittedEntries: + 3/12 EntryNormal "" + Messages: + 1->2 MsgAppResp Term:3 Log:0/12 +> 3 handling Ready + Ready MustSync=false: + HardState Term:3 Commit:12 + CommittedEntries: + 3/12 EntryNormal "" + Messages: + 3->2 MsgAppResp Term:3 Log:0/12 +> 2 receiving messages + 1->2 MsgAppResp Term:3 Log:0/12 + 3->2 MsgAppResp Term:3 Log:0/12 diff --git a/testdata/prevote.txt b/testdata/prevote.txt new file mode 100644 index 000000000000..d65103ed3ef3 --- /dev/null +++ b/testdata/prevote.txt @@ -0,0 +1,265 @@ +# Tests that PreVote prevents a node that is behind on the log from obtaining +# prevotes and calling an election. +# +# Also tests that a node that is up-to-date on its log can hold an election. +# Unlike the Raft thesis, the recent leader condition requires CheckQuorum +# and is not enforced with PreVote alone. + +log-level none +---- +ok + +add-nodes 3 voters=(1,2,3) index=10 prevote=true +---- +ok + +campaign 1 +---- +ok + +stabilize +---- +ok (quiet) + +log-level debug +---- +ok + +# Propose a command on 1 and replicate it to 2. +propose 1 prop_1 +---- +ok + +process-ready 1 +---- +Ready MustSync=true: +Entries: +1/12 EntryNormal "prop_1" +Messages: +1->2 MsgApp Term:1 Log:1/11 Commit:11 Entries:[1/12 EntryNormal "prop_1"] +1->3 MsgApp Term:1 Log:1/11 Commit:11 Entries:[1/12 EntryNormal "prop_1"] + +deliver-msgs 2 +---- +1->2 MsgApp Term:1 Log:1/11 Commit:11 Entries:[1/12 EntryNormal "prop_1"] + +process-ready 2 +---- +Ready MustSync=true: +Entries: +1/12 EntryNormal "prop_1" +Messages: +2->1 MsgAppResp Term:1 Log:0/12 + +# 3 is now behind on its log. Attempt to campaign. +raft-log 3 +---- +1/11 EntryNormal "" + +campaign 3 +---- +INFO 3 is starting a new election at term 1 +INFO 3 became pre-candidate at term 1 +INFO 3 [logterm: 1, index: 11] sent MsgPreVote request to 1 at term 1 +INFO 3 [logterm: 1, index: 11] sent MsgPreVote request to 2 at term 1 + +process-ready 3 +---- +Ready MustSync=false: +Lead:0 State:StatePreCandidate +Messages: +3->1 MsgPreVote Term:2 Log:1/11 +3->2 MsgPreVote Term:2 Log:1/11 +INFO 3 received MsgPreVoteResp from 3 at term 1 +INFO 3 has received 1 MsgPreVoteResp votes and 0 vote rejections + +deliver-msgs 1 2 +---- +2->1 MsgAppResp Term:1 Log:0/12 +3->1 MsgPreVote Term:2 Log:1/11 +INFO 1 [logterm: 1, index: 12, vote: 1] rejected MsgPreVote from 3 [logterm: 1, index: 11] at term 1 +3->2 MsgPreVote Term:2 Log:1/11 +INFO 2 [logterm: 1, index: 12, vote: 1] rejected MsgPreVote from 3 [logterm: 1, index: 11] at term 1 + +# 3 failed to campaign. Let the network stabilize. +stabilize +---- +> 1 handling Ready + Ready MustSync=false: + HardState Term:1 Vote:1 Commit:12 + CommittedEntries: + 1/12 EntryNormal "prop_1" + Messages: + 1->2 MsgApp Term:1 Log:1/12 Commit:12 + 1->3 MsgApp Term:1 Log:1/12 Commit:12 + 1->3 MsgPreVoteResp Term:1 Log:0/0 Rejected (Hint: 0) +> 2 handling Ready + Ready MustSync=false: + Messages: + 2->3 MsgPreVoteResp Term:1 Log:0/0 Rejected (Hint: 0) +> 2 receiving messages + 1->2 MsgApp Term:1 Log:1/12 Commit:12 +> 3 receiving messages + 1->3 MsgApp Term:1 Log:1/11 Commit:11 Entries:[1/12 EntryNormal "prop_1"] + INFO 3 became follower at term 1 + 1->3 MsgApp Term:1 Log:1/12 Commit:12 + 1->3 MsgPreVoteResp Term:1 Log:0/0 Rejected (Hint: 0) + 2->3 MsgPreVoteResp Term:1 Log:0/0 Rejected (Hint: 0) +> 2 handling Ready + Ready MustSync=false: + HardState Term:1 Vote:1 Commit:12 + CommittedEntries: + 1/12 EntryNormal "prop_1" + Messages: + 2->1 MsgAppResp Term:1 Log:0/12 +> 3 handling Ready + Ready MustSync=true: + Lead:1 State:StateFollower + HardState Term:1 Vote:1 Commit:12 + Entries: + 1/12 EntryNormal "prop_1" + CommittedEntries: + 1/12 EntryNormal "prop_1" + Messages: + 3->1 MsgAppResp Term:1 Log:0/12 + 3->1 MsgAppResp Term:1 Log:0/12 +> 1 receiving messages + 2->1 MsgAppResp Term:1 Log:0/12 + 3->1 MsgAppResp Term:1 Log:0/12 + 3->1 MsgAppResp Term:1 Log:0/12 + +# Let 2 campaign. It should succeed, since it's up-to-date on the log. +campaign 2 +---- +INFO 2 is starting a new election at term 1 +INFO 2 became pre-candidate at term 1 +INFO 2 [logterm: 1, index: 12] sent MsgPreVote request to 1 at term 1 +INFO 2 [logterm: 1, index: 12] sent MsgPreVote request to 3 at term 1 + +stabilize +---- +> 2 handling Ready + Ready MustSync=false: + Lead:0 State:StatePreCandidate + Messages: + 2->1 MsgPreVote Term:2 Log:1/12 + 2->3 MsgPreVote Term:2 Log:1/12 + INFO 2 received MsgPreVoteResp from 2 at term 1 + INFO 2 has received 1 MsgPreVoteResp votes and 0 vote rejections +> 1 receiving messages + 2->1 MsgPreVote Term:2 Log:1/12 + INFO 1 [logterm: 1, index: 12, vote: 1] cast MsgPreVote for 2 [logterm: 1, index: 12] at term 1 +> 3 receiving messages + 2->3 MsgPreVote Term:2 Log:1/12 + INFO 3 [logterm: 1, index: 12, vote: 1] cast MsgPreVote for 2 [logterm: 1, index: 12] at term 1 +> 1 handling Ready + Ready MustSync=false: + Messages: + 1->2 MsgPreVoteResp Term:2 Log:0/0 +> 3 handling Ready + Ready MustSync=false: + Messages: + 3->2 MsgPreVoteResp Term:2 Log:0/0 +> 2 receiving messages + 1->2 MsgPreVoteResp Term:2 Log:0/0 + INFO 2 received MsgPreVoteResp from 1 at term 1 + INFO 2 has received 2 MsgPreVoteResp votes and 0 vote rejections + INFO 2 became candidate at term 2 + INFO 2 [logterm: 1, index: 12] sent MsgVote request to 1 at term 2 + INFO 2 [logterm: 1, index: 12] sent MsgVote request to 3 at term 2 + 3->2 MsgPreVoteResp Term:2 Log:0/0 +> 2 handling Ready + Ready MustSync=true: + Lead:0 State:StateCandidate + HardState Term:2 Vote:2 Commit:12 + Messages: + 2->1 MsgVote Term:2 Log:1/12 + 2->3 MsgVote Term:2 Log:1/12 + INFO 2 received MsgVoteResp from 2 at term 2 + INFO 2 has received 1 MsgVoteResp votes and 0 vote rejections +> 1 receiving messages + 2->1 MsgVote Term:2 Log:1/12 + INFO 1 [term: 1] received a MsgVote message with higher term from 2 [term: 2] + INFO 1 became follower at term 2 + INFO 1 [logterm: 1, index: 12, vote: 0] cast MsgVote for 2 [logterm: 1, index: 12] at term 2 +> 3 receiving messages + 2->3 MsgVote Term:2 Log:1/12 + INFO 3 [term: 1] received a MsgVote message with higher term from 2 [term: 2] + INFO 3 became follower at term 2 + INFO 3 [logterm: 1, index: 12, vote: 0] cast MsgVote for 2 [logterm: 1, index: 12] at term 2 +> 1 handling Ready + Ready MustSync=true: + Lead:0 State:StateFollower + HardState Term:2 Vote:2 Commit:12 + Messages: + 1->2 MsgVoteResp Term:2 Log:0/0 +> 3 handling Ready + Ready MustSync=true: + Lead:0 State:StateFollower + HardState Term:2 Vote:2 Commit:12 + Messages: + 3->2 MsgVoteResp Term:2 Log:0/0 +> 2 receiving messages + 1->2 MsgVoteResp Term:2 Log:0/0 + INFO 2 received MsgVoteResp from 1 at term 2 + INFO 2 has received 2 MsgVoteResp votes and 0 vote rejections + INFO 2 became leader at term 2 + 3->2 MsgVoteResp Term:2 Log:0/0 +> 2 handling Ready + Ready MustSync=true: + Lead:2 State:StateLeader + Entries: + 2/13 EntryNormal "" + Messages: + 2->1 MsgApp Term:2 Log:1/12 Commit:12 Entries:[2/13 EntryNormal ""] + 2->3 MsgApp Term:2 Log:1/12 Commit:12 Entries:[2/13 EntryNormal ""] +> 1 receiving messages + 2->1 MsgApp Term:2 Log:1/12 Commit:12 Entries:[2/13 EntryNormal ""] +> 3 receiving messages + 2->3 MsgApp Term:2 Log:1/12 Commit:12 Entries:[2/13 EntryNormal ""] +> 1 handling Ready + Ready MustSync=true: + Lead:2 State:StateFollower + Entries: + 2/13 EntryNormal "" + Messages: + 1->2 MsgAppResp Term:2 Log:0/13 +> 3 handling Ready + Ready MustSync=true: + Lead:2 State:StateFollower + Entries: + 2/13 EntryNormal "" + Messages: + 3->2 MsgAppResp Term:2 Log:0/13 +> 2 receiving messages + 1->2 MsgAppResp Term:2 Log:0/13 + 3->2 MsgAppResp Term:2 Log:0/13 +> 2 handling Ready + Ready MustSync=false: + HardState Term:2 Vote:2 Commit:13 + CommittedEntries: + 2/13 EntryNormal "" + Messages: + 2->1 MsgApp Term:2 Log:2/13 Commit:13 + 2->3 MsgApp Term:2 Log:2/13 Commit:13 +> 1 receiving messages + 2->1 MsgApp Term:2 Log:2/13 Commit:13 +> 3 receiving messages + 2->3 MsgApp Term:2 Log:2/13 Commit:13 +> 1 handling Ready + Ready MustSync=false: + HardState Term:2 Vote:2 Commit:13 + CommittedEntries: + 2/13 EntryNormal "" + Messages: + 1->2 MsgAppResp Term:2 Log:0/13 +> 3 handling Ready + Ready MustSync=false: + HardState Term:2 Vote:2 Commit:13 + CommittedEntries: + 2/13 EntryNormal "" + Messages: + 3->2 MsgAppResp Term:2 Log:0/13 +> 2 receiving messages + 1->2 MsgAppResp Term:2 Log:0/13 + 3->2 MsgAppResp Term:2 Log:0/13 diff --git a/testdata/prevote_checkquorum.txt b/testdata/prevote_checkquorum.txt new file mode 100644 index 000000000000..939d6f360e80 --- /dev/null +++ b/testdata/prevote_checkquorum.txt @@ -0,0 +1,345 @@ +# Tests that PreVote+CheckQuorum prevents a node from obtaining prevotes if +# voters have heard from a leader recently. Also tests that a node is able to +# obtain prevotes if the voter hasn't heard from the leader in the past election +# timeout interval, or if a quorum of voters are precandidates. + +log-level none +---- +ok + +add-nodes 3 voters=(1,2,3) index=10 prevote=true checkquorum=true +---- +ok + +campaign 1 +---- +ok + +stabilize +---- +ok (quiet) + +log-level debug +---- +ok + +# 2 should fail to campaign, leaving 1's leadership alone. +campaign 2 +---- +INFO 2 is starting a new election at term 1 +INFO 2 became pre-candidate at term 1 +INFO 2 [logterm: 1, index: 11] sent MsgPreVote request to 1 at term 1 +INFO 2 [logterm: 1, index: 11] sent MsgPreVote request to 3 at term 1 + +stabilize +---- +> 2 handling Ready + Ready MustSync=false: + Lead:0 State:StatePreCandidate + Messages: + 2->1 MsgPreVote Term:2 Log:1/11 + 2->3 MsgPreVote Term:2 Log:1/11 + INFO 2 received MsgPreVoteResp from 2 at term 1 + INFO 2 has received 1 MsgPreVoteResp votes and 0 vote rejections +> 1 receiving messages + 2->1 MsgPreVote Term:2 Log:1/11 + INFO 1 [logterm: 1, index: 11, vote: 1] ignored MsgPreVote from 2 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) +> 3 receiving messages + 2->3 MsgPreVote Term:2 Log:1/11 + INFO 3 [logterm: 1, index: 11, vote: 1] ignored MsgPreVote from 2 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) + +# If 2 hasn't heard from the leader in the past election timeout, it should +# grant prevotes, allowing 3 to hold an election. +set-randomized-election-timeout 2 timeout=5 +---- +ok + +tick-election 2 +---- +ok + +campaign 3 +---- +INFO 3 is starting a new election at term 1 +INFO 3 became pre-candidate at term 1 +INFO 3 [logterm: 1, index: 11] sent MsgPreVote request to 1 at term 1 +INFO 3 [logterm: 1, index: 11] sent MsgPreVote request to 2 at term 1 + +process-ready 3 +---- +Ready MustSync=false: +Lead:0 State:StatePreCandidate +Messages: +3->1 MsgPreVote Term:2 Log:1/11 +3->2 MsgPreVote Term:2 Log:1/11 +INFO 3 received MsgPreVoteResp from 3 at term 1 +INFO 3 has received 1 MsgPreVoteResp votes and 0 vote rejections + +deliver-msgs 2 +---- +3->2 MsgPreVote Term:2 Log:1/11 +INFO 2 [logterm: 1, index: 11, vote: 1] cast MsgPreVote for 3 [logterm: 1, index: 11] at term 1 + +process-ready 2 +---- +Ready MustSync=false: +Messages: +2->3 MsgPreVoteResp Term:2 Log:0/0 + +stabilize +---- +> 1 receiving messages + 3->1 MsgPreVote Term:2 Log:1/11 + INFO 1 [logterm: 1, index: 11, vote: 1] ignored MsgPreVote from 3 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) +> 3 receiving messages + 2->3 MsgPreVoteResp Term:2 Log:0/0 + INFO 3 received MsgPreVoteResp from 2 at term 1 + INFO 3 has received 2 MsgPreVoteResp votes and 0 vote rejections + INFO 3 became candidate at term 2 + INFO 3 [logterm: 1, index: 11] sent MsgVote request to 1 at term 2 + INFO 3 [logterm: 1, index: 11] sent MsgVote request to 2 at term 2 +> 3 handling Ready + Ready MustSync=true: + Lead:0 State:StateCandidate + HardState Term:2 Vote:3 Commit:11 + Messages: + 3->1 MsgVote Term:2 Log:1/11 + 3->2 MsgVote Term:2 Log:1/11 + INFO 3 received MsgVoteResp from 3 at term 2 + INFO 3 has received 1 MsgVoteResp votes and 0 vote rejections +> 1 receiving messages + 3->1 MsgVote Term:2 Log:1/11 + INFO 1 [logterm: 1, index: 11, vote: 1] ignored MsgVote from 3 [logterm: 1, index: 11] at term 1: lease is not expired (remaining ticks: 3) +> 2 receiving messages + 3->2 MsgVote Term:2 Log:1/11 + INFO 2 [term: 1] received a MsgVote message with higher term from 3 [term: 2] + INFO 2 became follower at term 2 + INFO 2 [logterm: 1, index: 11, vote: 0] cast MsgVote for 3 [logterm: 1, index: 11] at term 2 +> 2 handling Ready + Ready MustSync=true: + Lead:0 State:StateFollower + HardState Term:2 Vote:3 Commit:11 + Messages: + 2->3 MsgVoteResp Term:2 Log:0/0 +> 3 receiving messages + 2->3 MsgVoteResp Term:2 Log:0/0 + INFO 3 received MsgVoteResp from 2 at term 2 + INFO 3 has received 2 MsgVoteResp votes and 0 vote rejections + INFO 3 became leader at term 2 +> 3 handling Ready + Ready MustSync=true: + Lead:3 State:StateLeader + Entries: + 2/12 EntryNormal "" + Messages: + 3->1 MsgApp Term:2 Log:1/11 Commit:11 Entries:[2/12 EntryNormal ""] + 3->2 MsgApp Term:2 Log:1/11 Commit:11 Entries:[2/12 EntryNormal ""] +> 1 receiving messages + 3->1 MsgApp Term:2 Log:1/11 Commit:11 Entries:[2/12 EntryNormal ""] + INFO 1 [term: 1] received a MsgApp message with higher term from 3 [term: 2] + INFO 1 became follower at term 2 +> 2 receiving messages + 3->2 MsgApp Term:2 Log:1/11 Commit:11 Entries:[2/12 EntryNormal ""] +> 1 handling Ready + Ready MustSync=true: + Lead:3 State:StateFollower + HardState Term:2 Commit:11 + Entries: + 2/12 EntryNormal "" + Messages: + 1->3 MsgAppResp Term:2 Log:0/12 +> 2 handling Ready + Ready MustSync=true: + Lead:3 State:StateFollower + Entries: + 2/12 EntryNormal "" + Messages: + 2->3 MsgAppResp Term:2 Log:0/12 +> 3 receiving messages + 1->3 MsgAppResp Term:2 Log:0/12 + 2->3 MsgAppResp Term:2 Log:0/12 +> 3 handling Ready + Ready MustSync=false: + HardState Term:2 Vote:3 Commit:12 + CommittedEntries: + 2/12 EntryNormal "" + Messages: + 3->1 MsgApp Term:2 Log:2/12 Commit:12 + 3->2 MsgApp Term:2 Log:2/12 Commit:12 +> 1 receiving messages + 3->1 MsgApp Term:2 Log:2/12 Commit:12 +> 2 receiving messages + 3->2 MsgApp Term:2 Log:2/12 Commit:12 +> 1 handling Ready + Ready MustSync=false: + HardState Term:2 Commit:12 + CommittedEntries: + 2/12 EntryNormal "" + Messages: + 1->3 MsgAppResp Term:2 Log:0/12 +> 2 handling Ready + Ready MustSync=false: + HardState Term:2 Vote:3 Commit:12 + CommittedEntries: + 2/12 EntryNormal "" + Messages: + 2->3 MsgAppResp Term:2 Log:0/12 +> 3 receiving messages + 1->3 MsgAppResp Term:2 Log:0/12 + 2->3 MsgAppResp Term:2 Log:0/12 + +# Node 3 is now the leader. Even though the leader is active, nodes 1 and 2 can +# still win a prevote and election if they both explicitly campaign, since the +# PreVote+CheckQuorum recent leader condition only applies to follower voters. +# This is beneficial, because it allows a quorum of nodes to replace a leader +# when they have strong reason to believe that it's dead, despite having heard +# from it recently. +# +# We first let 1 lose an election, as we'd otherwise get a tie. +campaign 1 +---- +INFO 1 is starting a new election at term 2 +INFO 1 became pre-candidate at term 2 +INFO 1 [logterm: 2, index: 12] sent MsgPreVote request to 2 at term 2 +INFO 1 [logterm: 2, index: 12] sent MsgPreVote request to 3 at term 2 + +stabilize +---- +> 1 handling Ready + Ready MustSync=false: + Lead:0 State:StatePreCandidate + Messages: + 1->2 MsgPreVote Term:3 Log:2/12 + 1->3 MsgPreVote Term:3 Log:2/12 + INFO 1 received MsgPreVoteResp from 1 at term 2 + INFO 1 has received 1 MsgPreVoteResp votes and 0 vote rejections +> 2 receiving messages + 1->2 MsgPreVote Term:3 Log:2/12 + INFO 2 [logterm: 2, index: 12, vote: 3] ignored MsgPreVote from 1 [logterm: 2, index: 12] at term 2: lease is not expired (remaining ticks: 3) +> 3 receiving messages + 1->3 MsgPreVote Term:3 Log:2/12 + INFO 3 [logterm: 2, index: 12, vote: 3] ignored MsgPreVote from 1 [logterm: 2, index: 12] at term 2: lease is not expired (remaining ticks: 3) + +campaign 2 +---- +INFO 2 is starting a new election at term 2 +INFO 2 became pre-candidate at term 2 +INFO 2 [logterm: 2, index: 12] sent MsgPreVote request to 1 at term 2 +INFO 2 [logterm: 2, index: 12] sent MsgPreVote request to 3 at term 2 + +stabilize +---- +> 2 handling Ready + Ready MustSync=false: + Lead:0 State:StatePreCandidate + Messages: + 2->1 MsgPreVote Term:3 Log:2/12 + 2->3 MsgPreVote Term:3 Log:2/12 + INFO 2 received MsgPreVoteResp from 2 at term 2 + INFO 2 has received 1 MsgPreVoteResp votes and 0 vote rejections +> 1 receiving messages + 2->1 MsgPreVote Term:3 Log:2/12 + INFO 1 [logterm: 2, index: 12, vote: 0] cast MsgPreVote for 2 [logterm: 2, index: 12] at term 2 +> 3 receiving messages + 2->3 MsgPreVote Term:3 Log:2/12 + INFO 3 [logterm: 2, index: 12, vote: 3] ignored MsgPreVote from 2 [logterm: 2, index: 12] at term 2: lease is not expired (remaining ticks: 3) +> 1 handling Ready + Ready MustSync=false: + Messages: + 1->2 MsgPreVoteResp Term:3 Log:0/0 +> 2 receiving messages + 1->2 MsgPreVoteResp Term:3 Log:0/0 + INFO 2 received MsgPreVoteResp from 1 at term 2 + INFO 2 has received 2 MsgPreVoteResp votes and 0 vote rejections + INFO 2 became candidate at term 3 + INFO 2 [logterm: 2, index: 12] sent MsgVote request to 1 at term 3 + INFO 2 [logterm: 2, index: 12] sent MsgVote request to 3 at term 3 +> 2 handling Ready + Ready MustSync=true: + Lead:0 State:StateCandidate + HardState Term:3 Vote:2 Commit:12 + Messages: + 2->1 MsgVote Term:3 Log:2/12 + 2->3 MsgVote Term:3 Log:2/12 + INFO 2 received MsgVoteResp from 2 at term 3 + INFO 2 has received 1 MsgVoteResp votes and 0 vote rejections +> 1 receiving messages + 2->1 MsgVote Term:3 Log:2/12 + INFO 1 [term: 2] received a MsgVote message with higher term from 2 [term: 3] + INFO 1 became follower at term 3 + INFO 1 [logterm: 2, index: 12, vote: 0] cast MsgVote for 2 [logterm: 2, index: 12] at term 3 +> 3 receiving messages + 2->3 MsgVote Term:3 Log:2/12 + INFO 3 [logterm: 2, index: 12, vote: 3] ignored MsgVote from 2 [logterm: 2, index: 12] at term 2: lease is not expired (remaining ticks: 3) +> 1 handling Ready + Ready MustSync=true: + Lead:0 State:StateFollower + HardState Term:3 Vote:2 Commit:12 + Messages: + 1->2 MsgVoteResp Term:3 Log:0/0 +> 2 receiving messages + 1->2 MsgVoteResp Term:3 Log:0/0 + INFO 2 received MsgVoteResp from 1 at term 3 + INFO 2 has received 2 MsgVoteResp votes and 0 vote rejections + INFO 2 became leader at term 3 +> 2 handling Ready + Ready MustSync=true: + Lead:2 State:StateLeader + Entries: + 3/13 EntryNormal "" + Messages: + 2->1 MsgApp Term:3 Log:2/12 Commit:12 Entries:[3/13 EntryNormal ""] + 2->3 MsgApp Term:3 Log:2/12 Commit:12 Entries:[3/13 EntryNormal ""] +> 1 receiving messages + 2->1 MsgApp Term:3 Log:2/12 Commit:12 Entries:[3/13 EntryNormal ""] +> 3 receiving messages + 2->3 MsgApp Term:3 Log:2/12 Commit:12 Entries:[3/13 EntryNormal ""] + INFO 3 [term: 2] received a MsgApp message with higher term from 2 [term: 3] + INFO 3 became follower at term 3 +> 1 handling Ready + Ready MustSync=true: + Lead:2 State:StateFollower + Entries: + 3/13 EntryNormal "" + Messages: + 1->2 MsgAppResp Term:3 Log:0/13 +> 3 handling Ready + Ready MustSync=true: + Lead:2 State:StateFollower + HardState Term:3 Commit:12 + Entries: + 3/13 EntryNormal "" + Messages: + 3->2 MsgAppResp Term:3 Log:0/13 +> 2 receiving messages + 1->2 MsgAppResp Term:3 Log:0/13 + 3->2 MsgAppResp Term:3 Log:0/13 +> 2 handling Ready + Ready MustSync=false: + HardState Term:3 Vote:2 Commit:13 + CommittedEntries: + 3/13 EntryNormal "" + Messages: + 2->1 MsgApp Term:3 Log:3/13 Commit:13 + 2->3 MsgApp Term:3 Log:3/13 Commit:13 +> 1 receiving messages + 2->1 MsgApp Term:3 Log:3/13 Commit:13 +> 3 receiving messages + 2->3 MsgApp Term:3 Log:3/13 Commit:13 +> 1 handling Ready + Ready MustSync=false: + HardState Term:3 Vote:2 Commit:13 + CommittedEntries: + 3/13 EntryNormal "" + Messages: + 1->2 MsgAppResp Term:3 Log:0/13 +> 3 handling Ready + Ready MustSync=false: + HardState Term:3 Commit:13 + CommittedEntries: + 3/13 EntryNormal "" + Messages: + 3->2 MsgAppResp Term:3 Log:0/13 +> 2 receiving messages + 1->2 MsgAppResp Term:3 Log:0/13 + 3->2 MsgAppResp Term:3 Log:0/13