Commit Graph

368 Commits (2f30e9ad7fc533b258b95d2fc95db5bbb3a7514f)

Author SHA1 Message Date
Tobias Schottdorf b9c051e7a7 raftpb: clean up naming in ConfChange 2019-07-23 10:40:03 +02:00
Tobias Schottdorf b67303c6a2 raft: allow use of joint quorums
This change introduces joint quorums by changing the Node and RawNode
API to accept pb.ConfChangeV2 (on top of pb.ConfChange).

pb.ConfChange continues to work as today: it allows carrying out a
single configuration change. A pb.ConfChange proposal gets added to
the Raft log as such and is thus also observed by the app during Ready
handling, and fed back to ApplyConfChange.

ConfChangeV2 allows joint configuration changes but will continue to
carry out configuration changes in "one phase" (i.e. without ever
entering a joint config) when this is possible.
2019-07-23 10:40:03 +02:00
Tobias Schottdorf 88f5561733 raft: use ConfChangeSingle internally 2019-07-23 10:39:48 +02:00
Jingyi Hu 233be58056
Merge pull request #10839 from needkane/pr
raft: update log info and annotation
2019-07-18 23:26:44 -07:00
Tobias Schottdorf aa158f36b9 raft: internally support joint consensus
This commit introduces machinery to safely apply joint consensus
configuration changes to Raft.

The main contribution is the new package, `confchange`, which offers
the primitives `Simple`, `EnterJoint`, and `LeaveJoint`.

The first two take a list of configuration changes. `Simple` only
declares success if these configuration changes (applied atomically)
change the set of voters by at most one (i.e. it's fine to add or
remove any number of learners, but change only one voter). `EnterJoint`
makes the configuration joint and then applies the changes to it, in
preparation of the caller returning later and transitioning out of the
joint config into the final desired configuration via `LeaveJoint()`.

This commit streamlines the conversion between voters and learners, which
is now generally allowed whenever the above conditions are upheld (i.e.
it's not possible to demote a voter and add a new voter in the context
of a Simple configuration change, but it is possible via EnterJoint).
Previously, we had the artificial restriction that a voter could not be
demoted to a learner, but had to be removed first.
Even though demoting a learner is generally less useful than promoting
a learner (the latter is used to catch up future voters), demotions
could see use in improved handling of temporary node unavailability,
where it is desired to remove voting power from a down node, but to
preserve its data should it return.

An additional change that was made in this commit is to prevent the use
of empty commit quorums, which was previously possible but for no good
reason; this:

Closes #10884.

The work left to do in a future PR is to actually expose joint
configurations to the applications using Raft. This will entail mostly
API design and the addition of suitable testing, which to be carried
out ergonomically is likely to motivate a larger refactor.

Touches #7625.
2019-07-16 15:36:04 +02:00
Tobias Schottdorf 6f009d211f raft: allow voter to become learner through snapshot
At the time of writing, we don't allow configuration changes to change
voters to learners directly, but note that a snapshot may compress
multiple changes to the configuration into one: the voter could have
been removed, then readded as a learner and the snapshot reflects both
changes. In that case, a voter receives a snapshot telling it that it is
now a learner. In fact, the node has to accept that snapshot, or it is
permanently cut off from the Raft log.

I think this just wasn't realized in the original work, but this is just
my guess since there generally is very little rationale on the various
decisions made. I also generally haven't been able to figure out whether
the decision to prevent voters from becoming learners without first
having been removed was motivated by some particular concern, or if it
just wasn't deemed necessary. I suspect it is the latter because
demoting a voter seems perfectly safe.

See https://github.com/etcd-io/etcd/pull/8751#issuecomment-342028091.
2019-07-08 09:32:24 +02:00
Tobias Schottdorf 6697adfff8 raft/tracker: pull Voters and Learners into Config struct
This is helpful to quickly print the configuration log messages without
having to specify Voters and Learners separately.

It will also come in handy for joint quorums because it allows holding
on to voters and learners as a unit, which is useful for unit testing.
2019-07-03 21:26:42 +02:00
Tobias Schottdorf b171e1c78b raft: centralize configuration change application
Put all the logic related to applying a configuration change in one
place in preparation for adding joint consensus.

This inspired various TODOs.

I had to rewrite TestSnapshotSucceedViaAppResp since it was relying
on a snapshot applied to the leader, which is now prevented.
2019-07-03 21:26:42 +02:00
kane 4f7d83a249 raft: update log info and annotation 2019-07-02 23:43:56 -04:00
Tobias Schottdorf f9c2d00fb3 raft: extract 'tracker' package
Mechanically extract `progressTracker`, `Progress`, and `inflights`
to their own package named `tracker`. Add lots of comments in the
progress, and take the opportunity to rename and clarify various
fields.
2019-06-21 22:15:00 +02:00
Tobias Schottdorf e039629907 raft: use half-populated joint quorum
To ease a future transition into joint quorums, this commit removes the
previous "ad-hoc" majority-based quorum and vote computations with that
introduced in the `raft/quorum` package.

More specifically, the progressTracker now uses a quorum.JointConfig for
which the "second" majority quorum is always empty; in this case the
quorum behaves like the one quorum.MajorityConfig that is actually
present. Or, more briefly, this change is a no-op, but it will take the
busywork out of actually starting to make use of joint quorums in the
future.

On a side node, I suspect that this might've fixed a bug regarding the
read index though I haven't been able to explicitly come up with a
counter-example. The problem was that the acks collected for the read
index weren't taking into account membership changes, so they'd run the
danger of using acks from nodes since removed to claim that a quorum of
acks had been received. There's a chance that there isn't a
counter-example (the only guarantee extracted from the "quorum" is that
there isn't another leader, but even if there's another leader all that
matters is that that leader doesn't have a divergent history from the
stale leader in the hypothetical counter-example), but either way there
is morally a bug here that is now fixed because VoteCommitted doesn't
care about votes from members that are not voters known to the currently
active configuration.
2019-06-19 14:19:35 +02:00
Tobias Schottdorf 0384c587eb raft: rename makeP{RS,rogressTracker} 2019-06-19 14:19:35 +02:00
Tobias Schottdorf c844526002 raft: prevent learners from becoming leader
We were already taking some precautions against learners campaigning,
but there was no safeguard against an explicit call to `Campaign()`.
The newly added test also verifies that leadership transfers to
learners are ignored.
2019-06-17 09:20:45 +02:00
Gyuho Lee 34bd797e67 *: revert module import paths
Signed-off-by: Gyuho Lee <leegyuho@amazon.com>
2019-05-28 15:39:35 -07:00
Tobias Schottdorf 5dd45011d6 raft: rename prs to progressTracker 2019-05-21 16:03:36 +02:00
Tobias Schottdorf 02b0d80234 raft: remove quorum() dependency from readOnly
This now delegates the quorum computation to r.prs, which will allow
it to generalize in a straightforward way when etcd-io/etcd#7625 is
addressed.
2019-05-21 16:03:36 +02:00
Tobias Schottdorf 57a1b39fcd raft: avoid another call to quorum()
This particular caller just wanted to know whether it was in a single-voter
cluster configuration, which is now a question prs can answer.
2019-05-21 16:02:52 +02:00
Tobias Schottdorf bc828e939a raft: pull checkQuorumActive into prs
It's looking at each voter's Progress and needs to know how quorums
work, so this is the ideal new home for it.
2019-05-21 16:02:52 +02:00
Tobias Schottdorf a6f222e62d raft: establish an interface around vote counting
This cleans up the mechanical refactor in the last commit and will
help with etcd-io/etcd#7625 as well.
2019-05-21 16:02:52 +02:00
Tobias Schottdorf 26eaadb1d1 raft: move votes into prs
This is purely mechanical. Cleanup deferred to the next commit.
2019-05-21 16:02:52 +02:00
Tobias Schottdorf a11563737c raft: use progress tracker APIs in more places
This doesn't completely eliminate access to prs.nodes, but that's not
really necessary. This commit uses the existing APIs in a few more
places where it's convenient, and also sprinkles some assertions.
2019-05-21 16:02:52 +02:00
Tobias Schottdorf ea82b2b758 raft: move more methods onto the progress tracker
Continues what was initiated in the last commit.
2019-05-21 16:02:52 +02:00
Tobias Schottdorf dbac67e7a8 raft: extract progress tracking into own component
The Progress maps contain both the active configuration and information
about the replication status. By pulling it into its own component, this
becomes easier to unit test and also clarifies the code, which will see
changes as etcd-io/etcd#7625 is addressed.

More functionality will move into `prs` in self-contained follow-up commits.
2019-05-21 16:02:52 +02:00
Gyuho Lee e899023f3f
Merge pull request #10640 from shrajfr12/gomodulecompat
Fix module path to have the major version to comply with go modules specification.
2019-05-01 22:46:03 -07:00
Xiang Li 0bc219a91e
Merge pull request #10679 from nvanbenschoten/nvanbenschoten/commitAlloc
raft: Avoid allocation when boxing slice in maybeCommit
2019-04-30 10:55:16 -07:00
shivaramr 9150bf52d6 go modules: Fix module path version to include version number 2019-04-26 15:29:50 -07:00
Nathan VanBenschoten 24f35a9861 raft: avoid allocation of Raft entry due to logging
`raftpb.Entry.String` takes a pointer receiver, so calling it
on a loop variable was causing the variable to escape. Removing
the `.String()` call was enough to avoid the allocation, but
this also avoids a memory copy and prevents similar bugs.

This was responsible for 11.63% of total allocations in an
experiment with https://github.com/nvanbenschoten/raft-toy.
2019-04-26 14:56:31 -04:00
Nathan VanBenschoten 208b8a349c raft: Avoid allocation when boxing slice in maybeCommit
By boxing a heap-allocated slice header instead of the slice
header on the stack, we can avoid an allocation when passing
through the sort.Interface interface.

This was responsible for 26.61% of total allocations in an
experiment with https://github.com/nvanbenschoten/raft-toy.
2019-04-26 00:10:45 -04:00
Jingyi Hu 5088d70d69 raft: leader response to learner MsgReadIndex
Leader should check message sender after receiving MsgReadIndex, even
when raft quorum is 1. The message could be sent from learner node, and
leader should respond.
2019-03-28 16:14:32 -07:00
Sahdev P. Zala 56f1bce161 raft/doc: clarify the case of out of date term
Clarify the doc.

Fixes #10491
2019-03-26 14:00:24 -04:00
shawnli 23731bf9ba raft: set lead to none when becomePreCandidate 2018-12-28 19:57:26 +08:00
Tobias Schottdorf bfaae1ba46 raft: enter ProgressStateReplica immediately after snapshot
When a follower requires a snapshot and the snapshot is sent at the
committed (and last) index in an otherwise idle Raft group, the follower
would previously remain in ProgressStateProbe even though it had been
caught up completely.

In a busy Raft group this wouldn't be an issue since the next round of
MsgApp would update the state, but in an idle group there's nothing
that rectifies the status (since there's nothing to append or update).

The reason this matters is that the state is exposed through
`RaftStatus()`. Concretely, in CockroachDB, we use the Raft status to
make sharding decisions (since it's dangerous to make rapid changes
to a fragile Raft group), and had to work around this problem[1].

[1]: 91b11dae41/pkg/storage/split_delay_helper.go (L138-L141)
2018-12-06 11:09:59 +01:00
Tobias Schottdorf 5c209d66d2 raft: ensure leader is in ProgressStateReplicate
The leader perpetually kept itself in ProgressStateProbe even though of
course it has perfect knowledge of its log. This wasn't usually an issue
because it also doesn't care about its own Progress, but it's better to
keep this data correctly maintained, especially since this is part of
raft.Status and thus becomes visible to applications using the Raft
library.

(Concretely, in CockroachDB we use the Progress to inform log
truncations).
2018-11-23 17:57:36 +01:00
Andrew Werner e4af2be5bb raft: separate MaxCommittedSizePerReady config from MaxSizePerMsg
Prior to this change, MaxSizePerMsg was used both to cap the total byte size of
entries in messages as well as the total byte size of entries passed through
CommittedEntries in the Ready struct. This change adds a new Config parameter
MaxCommittedSizePerReady which defaults to MaxSizePerMsg and contols the second
of above descibed settings.
2018-11-14 09:59:09 -05:00
Tobias Schottdorf ad49c8fd98 raft: fix bug in unbounded log growth prevention mechanism
The previous code was using the proto-generated `Size()` method to
track the size of an incoming proposal at the leader. This includes
the Index and Term, which were mutated after the call to `Size()`
when appending to the log. Additionally, it was not taking into
account that an ignored configuration change would ignore the
original proposal and append an empty entry instead.

As a result, a fully committed Raft group could end up with a non-
zero tracked uncommitted Raft log counter that would eventually hit
the ceiling and drop all future proposals indiscriminately. It would
also immediately imply that proposals exceeding the threshold alone
would get refused (as the "first uncommitted proposal" gets special
treatment and is always allowed in).

Track only the size of the payload actually appended to the Raft log
instead.

For context, see:
https://github.com/cockroachdb/cockroach/issues/31618#issuecomment-431374938
2018-10-22 11:28:39 +02:00
Nathan VanBenschoten 73c20cc1b7 raft: Fix comment on sendHeartbeat 2018-10-14 00:03:43 -04:00
Nathan VanBenschoten f89b06dc6d raft: provide protection against unbounded Raft log growth
The suggested pattern for Raft proposals is that they be retried
periodically until they succeed. This turns out to be an issue
when a leader cannot commit entries because the leader will continue
to append re-proposed entries to its log without committing anything.
This can result in the uncommitted tail of a leader's log growing
without bound until it is able to commit entries.

This change add a safeguard to protect against this case where a
leader's log can grow without bound during loss of quorum scenarios.
It does so by introducing a new, optional ``MaxUncommittedEntriesSize
configuration. This config limits the max aggregate size of uncommitted
entries that may be appended to a leader's log. Once this limit
is exceeded, proposals will begin to return ErrProposalDropped
errors.

See cockroachdb/cockroach#27772
2018-10-13 23:25:05 -04:00
Gyuho Lee bb60f8ab1d raft: change import paths to "go.etcd.io/etcd"
Signed-off-by: Gyuho Lee <leegyuho@amazon.com>
2018-08-28 17:47:52 -07:00
Xiang Li 11dd0b583b
Merge pull request #9982 from bdarnell/pagination
raft: Introduce CommittedEntries pagination
2018-08-11 09:12:46 +08:00
Ben Darnell a9e7c1e11f raft: Make flow control more aggressive
We allow multiple in-flight append messages, but prior to this change
the only way we'd ever send them is if there is a steady stream of new
proposals. Catching up a follower that is far behind would be
unnecessarily slow (this is exacerbated by a quirk of CockroachDB's
use of raft which limits our ability to catch up via snapshot in some
cases).

See cockroachdb/cockroach#27983
2018-08-08 11:10:54 -04:00
Ben Darnell 0a670b7c9b raft: Introduce CommittedEntries pagination
The MaxSizePerMsg setting is now used to limit the size of
Ready.CommittedEntries. This prevents out-of-memory errors if the raft
log has become very large and commits all at once.
2018-08-07 12:54:34 -04:00
Nathan VanBenschoten 0a415cf0d6 raft: dont allocate slice and sort on every commit 2018-07-25 23:42:16 -04:00
Ben Darnell 20422c5b4d raft: Really avoid scanning raft log in becomeLeader
I meant to do this in #9073, but sent the PR before it was finished.
The last log index is known directly; there is no need to fetch any
entries here.
2018-06-26 15:29:51 -04:00
Xiang Li 357308bfcd
Merge pull request #9679 from lorneli/lorneli-raft-dev
raft: describe the purpose of lockedRand
2018-05-26 22:03:18 -07:00
lorneli a083282482 raft: describe the purpose of lockedRand
Struct lockedRand wraps rand.Rand with mutex lock because it's
accessed by multiple raft groups.
2018-05-26 21:59:24 +08:00
Jia Zhan d14b705355 raft: fix a few comments 2018-04-27 11:25:06 -07:00
Gyuho Lee 8aae8c1c9c raft: document disruptive rejoining server, add tests
Signed-off-by: Gyuho Lee <gyuhox@gmail.com>
2018-03-06 09:54:29 -08:00
Gyuho Lee 01db389ea8 raft: document why reuse candidate's term for vote response in stepCandidate
"stepCandidate" should reuse candidate's own term, not term in Message,
because pre-vote is requested with future term.

Signed-off-by: Gyuho Lee <gyuhox@gmail.com>
2018-02-21 16:11:01 -08:00
Gyuho Lee 38846c220a raft: use leader's term when candidate becomes follower
`raft.Step` already ensures that when `m.Term > r.Term`,
candidate reverts back to follower with its term being
reset with `m.Term`, thus it's always true that
`m.Term == r.Term` in `stepCandidate`.

This just makes `r.becomeFollower` calls consistent.

Signed-off-by: Gyuho Lee <gyuhox@gmail.com>
2018-02-21 16:10:52 -08:00
Gyuho Lee 2b7c12fb12 raft: reuse "last index" in "appendEntry"
No need to call "lastIndex" again.
"append" call already returns "lastIndex".

Signed-off-by: Gyuho Lee <gyuhox@gmail.com>
2018-02-05 21:26:45 -08:00