confchange.go 13 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424
  1. // Copyright 2019 The etcd Authors
  2. //
  3. // Licensed under the Apache License, Version 2.0 (the "License");
  4. // you may not use this file except in compliance with the License.
  5. // You may obtain a copy of the License at
  6. //
  7. // http://www.apache.org/licenses/LICENSE-2.0
  8. //
  9. // Unless required by applicable law or agreed to in writing, software
  10. // distributed under the License is distributed on an "AS IS" BASIS,
  11. // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  12. // See the License for the specific language governing permissions and
  13. // limitations under the License.
  14. package confchange
  15. import (
  16. "errors"
  17. "fmt"
  18. "strings"
  19. "go.etcd.io/etcd/raft/quorum"
  20. pb "go.etcd.io/etcd/raft/raftpb"
  21. "go.etcd.io/etcd/raft/tracker"
  22. )
  23. // Changer facilitates configuration changes. It exposes methods to handle
  24. // simple and joint consensus while performing the proper validation that allows
  25. // refusing invalid configuration changes before they affect the active
  26. // configuration.
  27. type Changer struct {
  28. Tracker tracker.ProgressTracker
  29. LastIndex uint64
  30. }
  31. // EnterJoint verifies that the outgoing (=right) majority config of the joint
  32. // config is empty and initializes it with a copy of the incoming (=left)
  33. // majority config. That is, it transitions from
  34. //
  35. // (1 2 3)&&()
  36. // to
  37. // (1 2 3)&&(1 2 3).
  38. //
  39. // The supplied changes are then applied to the incoming majority config,
  40. // resulting in a joint configuration that in terms of the Raft thesis[1]
  41. // (Section 4.3) corresponds to `C_{new,old}`.
  42. //
  43. // [1]: https://github.com/ongardie/dissertation/blob/master/online-trim.pdf
  44. func (c Changer) EnterJoint(autoLeave bool, ccs ...pb.ConfChangeSingle) (tracker.Config, tracker.ProgressMap, error) {
  45. cfg, prs, err := c.checkAndCopy()
  46. if err != nil {
  47. return c.err(err)
  48. }
  49. if joint(cfg) {
  50. err := errors.New("config is already joint")
  51. return c.err(err)
  52. }
  53. if len(incoming(cfg.Voters)) == 0 {
  54. // We allow adding nodes to an empty config for convenience (testing and
  55. // bootstrap), but you can't enter a joint state.
  56. err := errors.New("can't make a zero-voter config joint")
  57. return c.err(err)
  58. }
  59. // Clear the outgoing config.
  60. {
  61. *outgoingPtr(&cfg.Voters) = quorum.MajorityConfig{}
  62. }
  63. // Copy incoming to outgoing.
  64. for id := range incoming(cfg.Voters) {
  65. outgoing(cfg.Voters)[id] = struct{}{}
  66. }
  67. if err := c.apply(&cfg, prs, ccs...); err != nil {
  68. return c.err(err)
  69. }
  70. cfg.AutoLeave = autoLeave
  71. return checkAndReturn(cfg, prs)
  72. }
  73. // LeaveJoint transitions out of a joint configuration. It is an error to call
  74. // this method if the configuration is not joint, i.e. if the outgoing majority
  75. // config Voters[1] is empty.
  76. //
  77. // The outgoing majority config of the joint configuration will be removed,
  78. // that is, the incoming config is promoted as the sole decision maker. In the
  79. // notation of the Raft thesis[1] (Section 4.3), this method transitions from
  80. // `C_{new,old}` into `C_new`.
  81. //
  82. // At the same time, any staged learners (LearnersNext) the addition of which
  83. // was held back by an overlapping voter in the former outgoing config will be
  84. // inserted into Learners.
  85. //
  86. // [1]: https://github.com/ongardie/dissertation/blob/master/online-trim.pdf
  87. func (c Changer) LeaveJoint() (tracker.Config, tracker.ProgressMap, error) {
  88. cfg, prs, err := c.checkAndCopy()
  89. if err != nil {
  90. return c.err(err)
  91. }
  92. if !joint(cfg) {
  93. err := errors.New("can't leave a non-joint config")
  94. return c.err(err)
  95. }
  96. if len(outgoing(cfg.Voters)) == 0 {
  97. err := fmt.Errorf("configuration is not joint: %v", cfg)
  98. return c.err(err)
  99. }
  100. for id := range cfg.LearnersNext {
  101. nilAwareAdd(&cfg.Learners, id)
  102. prs[id].IsLearner = true
  103. }
  104. cfg.LearnersNext = nil
  105. for id := range outgoing(cfg.Voters) {
  106. _, isVoter := incoming(cfg.Voters)[id]
  107. _, isLearner := cfg.Learners[id]
  108. if !isVoter && !isLearner {
  109. delete(prs, id)
  110. }
  111. }
  112. *outgoingPtr(&cfg.Voters) = nil
  113. cfg.AutoLeave = false
  114. return checkAndReturn(cfg, prs)
  115. }
  116. // Simple carries out a series of configuration changes that (in aggregate)
  117. // mutates the incoming majority config Voters[0] by at most one. This method
  118. // will return an error if that is not the case, if the resulting quorum is
  119. // zero, or if the configuration is in a joint state (i.e. if there is an
  120. // outgoing configuration).
  121. func (c Changer) Simple(ccs ...pb.ConfChangeSingle) (tracker.Config, tracker.ProgressMap, error) {
  122. cfg, prs, err := c.checkAndCopy()
  123. if err != nil {
  124. return c.err(err)
  125. }
  126. if joint(cfg) {
  127. err := errors.New("can't apply simple config change in joint config")
  128. return c.err(err)
  129. }
  130. if err := c.apply(&cfg, prs, ccs...); err != nil {
  131. return c.err(err)
  132. }
  133. if n := symdiff(incoming(c.Tracker.Voters), incoming(cfg.Voters)); n > 1 {
  134. return tracker.Config{}, nil, errors.New("more than one voter changed without entering joint config")
  135. }
  136. if err := checkInvariants(cfg, prs); err != nil {
  137. return tracker.Config{}, tracker.ProgressMap{}, nil
  138. }
  139. return checkAndReturn(cfg, prs)
  140. }
  141. // apply a change to the configuration. By convention, changes to voters are
  142. // always made to the incoming majority config Voters[0]. Voters[1] is either
  143. // empty or preserves the outgoing majority configuration while in a joint state.
  144. func (c Changer) apply(cfg *tracker.Config, prs tracker.ProgressMap, ccs ...pb.ConfChangeSingle) error {
  145. for _, cc := range ccs {
  146. if cc.NodeID == 0 {
  147. // etcd replaces the NodeID with zero if it decides (downstream of
  148. // raft) to not apply a change, so we have to have explicit code
  149. // here to ignore these.
  150. continue
  151. }
  152. switch cc.Type {
  153. case pb.ConfChangeAddNode:
  154. c.makeVoter(cfg, prs, cc.NodeID)
  155. case pb.ConfChangeAddLearnerNode:
  156. c.makeLearner(cfg, prs, cc.NodeID)
  157. case pb.ConfChangeRemoveNode:
  158. c.remove(cfg, prs, cc.NodeID)
  159. case pb.ConfChangeUpdateNode:
  160. default:
  161. return fmt.Errorf("unexpected conf type %d", cc.Type)
  162. }
  163. }
  164. if len(incoming(cfg.Voters)) == 0 {
  165. return errors.New("removed all voters")
  166. }
  167. return nil
  168. }
  169. // makeVoter adds or promotes the given ID to be a voter in the incoming
  170. // majority config.
  171. func (c Changer) makeVoter(cfg *tracker.Config, prs tracker.ProgressMap, id uint64) {
  172. pr := prs[id]
  173. if pr == nil {
  174. c.initProgress(cfg, prs, id, false /* isLearner */)
  175. return
  176. }
  177. pr.IsLearner = false
  178. nilAwareDelete(&cfg.Learners, id)
  179. nilAwareDelete(&cfg.LearnersNext, id)
  180. incoming(cfg.Voters)[id] = struct{}{}
  181. return
  182. }
  183. // makeLearner makes the given ID a learner or stages it to be a learner once
  184. // an active joint configuration is exited.
  185. //
  186. // The former happens when the peer is not a part of the outgoing config, in
  187. // which case we either add a new learner or demote a voter in the incoming
  188. // config.
  189. //
  190. // The latter case occurs when the configuration is joint and the peer is a
  191. // voter in the outgoing config. In that case, we do not want to add the peer
  192. // as a learner because then we'd have to track a peer as a voter and learner
  193. // simultaneously. Instead, we add the learner to LearnersNext, so that it will
  194. // be added to Learners the moment the outgoing config is removed by
  195. // LeaveJoint().
  196. func (c Changer) makeLearner(cfg *tracker.Config, prs tracker.ProgressMap, id uint64) {
  197. pr := prs[id]
  198. if pr == nil {
  199. c.initProgress(cfg, prs, id, true /* isLearner */)
  200. return
  201. }
  202. if pr.IsLearner {
  203. return
  204. }
  205. // Remove any existing voter in the incoming config...
  206. c.remove(cfg, prs, id)
  207. // ... but save the Progress.
  208. prs[id] = pr
  209. // Use LearnersNext if we can't add the learner to Learners directly, i.e.
  210. // if the peer is still tracked as a voter in the outgoing config. It will
  211. // be turned into a learner in LeaveJoint().
  212. //
  213. // Otherwise, add a regular learner right away.
  214. if _, onRight := outgoing(cfg.Voters)[id]; onRight {
  215. nilAwareAdd(&cfg.LearnersNext, id)
  216. } else {
  217. pr.IsLearner = true
  218. nilAwareAdd(&cfg.Learners, id)
  219. }
  220. }
  221. // remove this peer as a voter or learner from the incoming config.
  222. func (c Changer) remove(cfg *tracker.Config, prs tracker.ProgressMap, id uint64) {
  223. if _, ok := prs[id]; !ok {
  224. return
  225. }
  226. delete(incoming(cfg.Voters), id)
  227. nilAwareDelete(&cfg.Learners, id)
  228. nilAwareDelete(&cfg.LearnersNext, id)
  229. // If the peer is still a voter in the outgoing config, keep the Progress.
  230. if _, onRight := outgoing(cfg.Voters)[id]; !onRight {
  231. delete(prs, id)
  232. }
  233. }
  234. // initProgress initializes a new progress for the given node or learner.
  235. func (c Changer) initProgress(cfg *tracker.Config, prs tracker.ProgressMap, id uint64, isLearner bool) {
  236. if !isLearner {
  237. incoming(cfg.Voters)[id] = struct{}{}
  238. } else {
  239. nilAwareAdd(&cfg.Learners, id)
  240. }
  241. prs[id] = &tracker.Progress{
  242. // We initialize Progress.Next with lastIndex+1 so that the peer will be
  243. // probed without an index first.
  244. //
  245. // TODO(tbg): verify that, this is just my best guess.
  246. Next: c.LastIndex + 1,
  247. Match: 0,
  248. Inflights: tracker.NewInflights(c.Tracker.MaxInflight),
  249. IsLearner: isLearner,
  250. // When a node is first added, we should mark it as recently active.
  251. // Otherwise, CheckQuorum may cause us to step down if it is invoked
  252. // before the added node has had a chance to communicate with us.
  253. RecentActive: true,
  254. }
  255. }
  256. // checkInvariants makes sure that the config and progress are compatible with
  257. // each other. This is used to check both what the Changer is initialized with,
  258. // as well as what it returns.
  259. func checkInvariants(cfg tracker.Config, prs tracker.ProgressMap) error {
  260. // NB: intentionally allow the empty config. In production we'll never see a
  261. // non-empty config (we prevent it from being created) but we will need to
  262. // be able to *create* an initial config, for example during bootstrap (or
  263. // during tests). Instead of having to hand-code this, we allow
  264. // transitioning from an empty config into any other legal and non-empty
  265. // config.
  266. for _, ids := range []map[uint64]struct{}{
  267. cfg.Voters.IDs(),
  268. cfg.Learners,
  269. cfg.LearnersNext,
  270. } {
  271. for id := range ids {
  272. if _, ok := prs[id]; !ok {
  273. return fmt.Errorf("no progress for %d", id)
  274. }
  275. }
  276. }
  277. // Any staged learner was staged because it could not be directly added due
  278. // to a conflicting voter in the outgoing config.
  279. for id := range cfg.LearnersNext {
  280. if _, ok := outgoing(cfg.Voters)[id]; !ok {
  281. return fmt.Errorf("%d is in LearnersNext, but not Voters[1]", id)
  282. }
  283. if prs[id].IsLearner {
  284. return fmt.Errorf("%d is in LearnersNext, but is already marked as learner", id)
  285. }
  286. }
  287. // Conversely Learners and Voters doesn't intersect at all.
  288. for id := range cfg.Learners {
  289. if _, ok := outgoing(cfg.Voters)[id]; ok {
  290. return fmt.Errorf("%d is in Learners and Voters[1]", id)
  291. }
  292. if _, ok := incoming(cfg.Voters)[id]; ok {
  293. return fmt.Errorf("%d is in Learners and Voters[0]", id)
  294. }
  295. if !prs[id].IsLearner {
  296. return fmt.Errorf("%d is in Learners, but is not marked as learner", id)
  297. }
  298. }
  299. if !joint(cfg) {
  300. // We enforce that empty maps are nil instead of zero.
  301. if outgoing(cfg.Voters) != nil {
  302. return fmt.Errorf("Voters[1] must be nil when not joint")
  303. }
  304. if cfg.LearnersNext != nil {
  305. return fmt.Errorf("LearnersNext must be nil when not joint")
  306. }
  307. if cfg.AutoLeave {
  308. return fmt.Errorf("AutoLeave must be false when not joint")
  309. }
  310. }
  311. return nil
  312. }
  313. // checkAndCopy copies the tracker's config and progress map (deeply enough for
  314. // the purposes of the Changer) and returns those copies. It returns an error
  315. // if checkInvariants does.
  316. func (c Changer) checkAndCopy() (tracker.Config, tracker.ProgressMap, error) {
  317. cfg := c.Tracker.Config.Clone()
  318. prs := tracker.ProgressMap{}
  319. for id, pr := range c.Tracker.Progress {
  320. // A shallow copy is enough because we only mutate the Learner field.
  321. ppr := *pr
  322. prs[id] = &ppr
  323. }
  324. return checkAndReturn(cfg, prs)
  325. }
  326. // checkAndReturn calls checkInvariants on the input and returns either the
  327. // resulting error or the input.
  328. func checkAndReturn(cfg tracker.Config, prs tracker.ProgressMap) (tracker.Config, tracker.ProgressMap, error) {
  329. if err := checkInvariants(cfg, prs); err != nil {
  330. return tracker.Config{}, tracker.ProgressMap{}, err
  331. }
  332. return cfg, prs, nil
  333. }
  334. // err returns zero values and an error.
  335. func (c Changer) err(err error) (tracker.Config, tracker.ProgressMap, error) {
  336. return tracker.Config{}, nil, err
  337. }
  338. // nilAwareAdd populates a map entry, creating the map if necessary.
  339. func nilAwareAdd(m *map[uint64]struct{}, id uint64) {
  340. if *m == nil {
  341. *m = map[uint64]struct{}{}
  342. }
  343. (*m)[id] = struct{}{}
  344. }
  345. // nilAwareDelete deletes from a map, nil'ing the map itself if it is empty after.
  346. func nilAwareDelete(m *map[uint64]struct{}, id uint64) {
  347. if *m == nil {
  348. return
  349. }
  350. delete(*m, id)
  351. if len(*m) == 0 {
  352. *m = nil
  353. }
  354. }
  355. // symdiff returns the count of the symmetric difference between the sets of
  356. // uint64s, i.e. len( (l - r) \union (r - l)).
  357. func symdiff(l, r map[uint64]struct{}) int {
  358. var n int
  359. pairs := [][2]quorum.MajorityConfig{
  360. {l, r}, // count elems in l but not in r
  361. {r, l}, // count elems in r but not in l
  362. }
  363. for _, p := range pairs {
  364. for id := range p[0] {
  365. if _, ok := p[1][id]; !ok {
  366. n++
  367. }
  368. }
  369. }
  370. return n
  371. }
  372. func joint(cfg tracker.Config) bool {
  373. return len(outgoing(cfg.Voters)) > 0
  374. }
  375. func incoming(voters quorum.JointConfig) quorum.MajorityConfig { return voters[0] }
  376. func outgoing(voters quorum.JointConfig) quorum.MajorityConfig { return voters[1] }
  377. func outgoingPtr(voters *quorum.JointConfig) *quorum.MajorityConfig { return &voters[1] }
  378. // Describe prints the type and NodeID of the configuration changes as a
  379. // space-delimited string.
  380. func Describe(ccs ...pb.ConfChangeSingle) string {
  381. var buf strings.Builder
  382. for _, cc := range ccs {
  383. if buf.Len() > 0 {
  384. buf.WriteByte(' ')
  385. }
  386. fmt.Fprintf(&buf, "%s(%d)", cc.Type, cc.NodeID)
  387. }
  388. return buf.String()
  389. }