Skip to content

Commit b2e4771

Browse files
committed
Extract raft functions into raft directory
Signed-off-by: Marek Siarkowicz <[email protected]>
1 parent b868ea1 commit b2e4771

File tree

11 files changed

+599
-481
lines changed

11 files changed

+599
-481
lines changed

server/etcdserver/adapters.go

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -85,5 +85,5 @@ func (s *serverVersionAdapter) UpdateStorageVersion(target semver.Version) error
8585
tx := s.be.BatchTx()
8686
tx.LockOutsideApply()
8787
defer tx.Unlock()
88-
return schema.UnsafeMigrate(s.lg, tx, s.r.storage, target)
88+
return schema.UnsafeMigrate(s.lg, tx, s.r.Storage, target)
8989
}

server/etcdserver/bootstrap.go

Lines changed: 3 additions & 38 deletions
Original file line numberDiff line numberDiff line change
@@ -41,6 +41,7 @@ import (
4141
"go.etcd.io/etcd/server/v3/etcdserver/api/v3discovery"
4242
"go.etcd.io/etcd/server/v3/etcdserver/cindex"
4343
servererrors "go.etcd.io/etcd/server/v3/etcdserver/errors"
44+
serverraft "go.etcd.io/etcd/server/v3/etcdserver/raft"
4445
serverstorage "go.etcd.io/etcd/server/v3/storage"
4546
"go.etcd.io/etcd/server/v3/storage/backend"
4647
"go.etcd.io/etcd/server/v3/storage/schema"
@@ -493,7 +494,7 @@ func bootstrapRaftFromCluster(cfg config.ServerConfig, cl *membership.RaftCluste
493494
return &bootstrappedRaft{
494495
lg: cfg.Logger,
495496
heartbeat: time.Duration(cfg.TickMs) * time.Millisecond,
496-
config: raftConfig(cfg, uint64(member.ID), s),
497+
config: serverraft.NewConfig(cfg, uint64(member.ID), s),
497498
peers: peers,
498499
storage: s,
499500
}
@@ -504,47 +505,11 @@ func bootstrapRaftFromWAL(cfg config.ServerConfig, bwal *bootstrappedWAL) *boots
504505
return &bootstrappedRaft{
505506
lg: cfg.Logger,
506507
heartbeat: time.Duration(cfg.TickMs) * time.Millisecond,
507-
config: raftConfig(cfg, uint64(bwal.meta.nodeID), s),
508+
config: serverraft.NewConfig(cfg, uint64(bwal.meta.nodeID), s),
508509
storage: s,
509510
}
510511
}
511512

512-
func raftConfig(cfg config.ServerConfig, id uint64, s *raft.MemoryStorage) *raft.Config {
513-
return &raft.Config{
514-
ID: id,
515-
ElectionTick: cfg.ElectionTicks,
516-
HeartbeatTick: 1,
517-
Storage: s,
518-
MaxSizePerMsg: maxSizePerMsg,
519-
MaxInflightMsgs: maxInflightMsgs,
520-
CheckQuorum: true,
521-
PreVote: cfg.PreVote,
522-
Logger: NewRaftLoggerZap(cfg.Logger.Named("raft")),
523-
}
524-
}
525-
526-
func (b *bootstrappedRaft) newRaftNode(ss *snap.Snapshotter, wal *wal.WAL, cl *membership.RaftCluster) *raftNode {
527-
var n raft.Node
528-
if len(b.peers) == 0 {
529-
n = raft.RestartNode(b.config)
530-
} else {
531-
n = raft.StartNode(b.config, b.peers)
532-
}
533-
raftStatusMu.Lock()
534-
raftStatus = n.Status
535-
raftStatusMu.Unlock()
536-
return newRaftNode(
537-
raftNodeConfig{
538-
lg: b.lg,
539-
isIDRemoved: func(id uint64) bool { return cl.IsIDRemoved(types.ID(id)) },
540-
Node: n,
541-
heartbeat: b.heartbeat,
542-
raftStorage: b.storage,
543-
storage: serverstorage.NewStorage(b.lg, wal, ss),
544-
},
545-
)
546-
}
547-
548513
func bootstrapWALFromSnapshot(cfg config.ServerConfig, snapshot *raftpb.Snapshot, ci cindex.ConsistentIndexer) *bootstrappedWAL {
549514
wal, st, ents, snap, meta := openWALFromSnapshot(cfg, snapshot)
550515
bwal := &bootstrappedWAL{

server/etcdserver/metrics.go

Lines changed: 0 additions & 37 deletions
Original file line numberDiff line numberDiff line change
@@ -26,24 +26,6 @@ import (
2626
)
2727

2828
var (
29-
hasLeader = prometheus.NewGauge(prometheus.GaugeOpts{
30-
Namespace: "etcd",
31-
Subsystem: "server",
32-
Name: "has_leader",
33-
Help: "Whether or not a leader exists. 1 is existence, 0 is not.",
34-
})
35-
isLeader = prometheus.NewGauge(prometheus.GaugeOpts{
36-
Namespace: "etcd",
37-
Subsystem: "server",
38-
Name: "is_leader",
39-
Help: "Whether or not this member is a leader. 1 if is, 0 otherwise.",
40-
})
41-
leaderChanges = prometheus.NewCounter(prometheus.CounterOpts{
42-
Namespace: "etcd",
43-
Subsystem: "server",
44-
Name: "leader_changes_seen_total",
45-
Help: "The number of leader changes seen.",
46-
})
4729
learnerPromoteFailed = prometheus.NewCounterVec(
4830
prometheus.CounterOpts{
4931
Namespace: "etcd",
@@ -59,24 +41,12 @@ var (
5941
Name: "learner_promote_successes",
6042
Help: "The total number of successful learner promotions while this member is leader.",
6143
})
62-
heartbeatSendFailures = prometheus.NewCounter(prometheus.CounterOpts{
63-
Namespace: "etcd",
64-
Subsystem: "server",
65-
Name: "heartbeat_send_failures_total",
66-
Help: "The total number of leader heartbeat send failures (likely overloaded from slow disk).",
67-
})
6844
applySnapshotInProgress = prometheus.NewGauge(prometheus.GaugeOpts{
6945
Namespace: "etcd",
7046
Subsystem: "server",
7147
Name: "snapshot_apply_in_progress_total",
7248
Help: "1 if the server is applying the incoming snapshot. 0 if none.",
7349
})
74-
proposalsCommitted = prometheus.NewGauge(prometheus.GaugeOpts{
75-
Namespace: "etcd",
76-
Subsystem: "server",
77-
Name: "proposals_committed_total",
78-
Help: "The total number of consensus proposals committed.",
79-
})
8050
proposalsApplied = prometheus.NewGauge(prometheus.GaugeOpts{
8151
Namespace: "etcd",
8252
Subsystem: "server",
@@ -162,20 +132,13 @@ var (
162132
)
163133

164134
func init() {
165-
prometheus.MustRegister(hasLeader)
166-
prometheus.MustRegister(isLeader)
167-
prometheus.MustRegister(leaderChanges)
168-
prometheus.MustRegister(heartbeatSendFailures)
169135
prometheus.MustRegister(applySnapshotInProgress)
170-
prometheus.MustRegister(proposalsCommitted)
171136
prometheus.MustRegister(proposalsApplied)
172137
prometheus.MustRegister(proposalsPending)
173138
prometheus.MustRegister(proposalsFailed)
174139
prometheus.MustRegister(slowReadIndex)
175140
prometheus.MustRegister(readIndexFailed)
176141
prometheus.MustRegister(leaseExpired)
177-
prometheus.MustRegister(currentVersion)
178-
prometheus.MustRegister(currentGoVersion)
179142
prometheus.MustRegister(serverID)
180143
prometheus.MustRegister(serverFeatureEnabled)
181144
prometheus.MustRegister(learnerPromoteSucceed)

server/etcdserver/zap_raft.go renamed to server/etcdserver/raft/logger.go

Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -4,15 +4,15 @@
44
// you may not use this file except in compliance with the License.
55
// You may obtain a copy of the License at
66
//
7-
// http://www.apache.org/licenses/LICENSE-2.0
7+
// http://www.apache.org/licenses/LICENSE-2.0
88
//
99
// Unless required by applicable law or agreed to in writing, software
1010
// distributed under the License is distributed on an "AS IS" BASIS,
1111
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1212
// See the License for the specific language governing permissions and
1313
// limitations under the License.
1414

15-
package etcdserver
15+
package raft
1616

1717
import (
1818
"errors"
@@ -32,6 +32,7 @@ func NewRaftLogger(lcfg *zap.Config) (raft.Logger, error) {
3232
if err != nil {
3333
return nil, err
3434
}
35+
3536
return &zapRaftLogger{lg: lg, sugar: lg.Sugar()}, nil
3637
}
3738

server/etcdserver/zap_raft_test.go renamed to server/etcdserver/raft/logger_test.go

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -12,7 +12,7 @@
1212
// See the License for the specific language governing permissions and
1313
// limitations under the License.
1414

15-
package etcdserver
15+
package raft
1616

1717
import (
1818
"bytes"
@@ -67,7 +67,7 @@ func TestNewRaftLogger(t *testing.T) {
6767
if !bytes.Contains(data, []byte("etcd-logutil-2")) {
6868
t.Fatalf("can't find data in log %q", string(data))
6969
}
70-
if !bytes.Contains(data, []byte("zap_raft_test.go:")) {
70+
if !bytes.Contains(data, []byte("raft/logger_test.go:")) {
7171
t.Fatalf("unexpected caller; %q", string(data))
7272
}
7373
}

server/etcdserver/raft/metrics.go

Lines changed: 60 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,60 @@
1+
// Copyright 2015 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+
15+
package raft
16+
17+
import (
18+
"github.com/prometheus/client_golang/prometheus"
19+
)
20+
21+
var (
22+
hasLeader = prometheus.NewGauge(prometheus.GaugeOpts{
23+
Namespace: "etcd",
24+
Subsystem: "server",
25+
Name: "has_leader",
26+
Help: "Whether or not a leader exists. 1 is existence, 0 is not.",
27+
})
28+
isLeader = prometheus.NewGauge(prometheus.GaugeOpts{
29+
Namespace: "etcd",
30+
Subsystem: "server",
31+
Name: "is_leader",
32+
Help: "Whether or not this member is a leader. 1 if is, 0 otherwise.",
33+
})
34+
leaderChanges = prometheus.NewCounter(prometheus.CounterOpts{
35+
Namespace: "etcd",
36+
Subsystem: "server",
37+
Name: "leader_changes_seen_total",
38+
Help: "The number of leader changes seen.",
39+
})
40+
heartbeatSendFailures = prometheus.NewCounter(prometheus.CounterOpts{
41+
Namespace: "etcd",
42+
Subsystem: "server",
43+
Name: "heartbeat_send_failures_total",
44+
Help: "The total number of leader heartbeat send failures (likely overloaded from slow disk).",
45+
})
46+
proposalsCommitted = prometheus.NewGauge(prometheus.GaugeOpts{
47+
Namespace: "etcd",
48+
Subsystem: "server",
49+
Name: "proposals_committed_total",
50+
Help: "The total number of consensus proposals committed.",
51+
})
52+
)
53+
54+
func init() {
55+
prometheus.MustRegister(hasLeader)
56+
prometheus.MustRegister(isLeader)
57+
prometheus.MustRegister(leaderChanges)
58+
prometheus.MustRegister(heartbeatSendFailures)
59+
prometheus.MustRegister(proposalsCommitted)
60+
}

0 commit comments

Comments
 (0)