forked from dgraph-io/dgraph
/
raft.go
430 lines (380 loc) · 10.3 KB
/
raft.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
/*
* Copyright (C) 2017 Dgraph Labs, Inc. and Contributors
*
* This program is free software: you can redistribute it and/or modify
* it under the terms of the GNU Affero General Public License as published by
* the Free Software Foundation, either version 3 of the License, or
* (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU Affero General Public License for more details.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
package main
import (
"errors"
"math/rand"
"sync"
"time"
"google.golang.org/grpc"
"github.com/coreos/etcd/raft"
"github.com/coreos/etcd/raft/raftpb"
"github.com/dgraph-io/badger/y"
"gopkg.in/adibiarsotp/dgraph.v83/conn"
"gopkg.in/adibiarsotp/dgraph.v83/protos"
"gopkg.in/adibiarsotp/dgraph.v83/raftwal"
"gopkg.in/adibiarsotp/dgraph.v83/x"
"golang.org/x/net/context"
"golang.org/x/net/trace"
)
type proposalCtx struct {
ch chan error
ctx context.Context
}
type proposals struct {
sync.RWMutex
ids map[uint32]*proposalCtx
}
func (p *proposals) Store(pid uint32, pctx *proposalCtx) bool {
if pid == 0 {
return false
}
p.Lock()
defer p.Unlock()
if p.ids == nil {
p.ids = make(map[uint32]*proposalCtx)
}
if _, has := p.ids[pid]; has {
return false
}
p.ids[pid] = pctx
return true
}
func (p *proposals) Done(pid uint32, err error) {
p.Lock()
defer p.Unlock()
pd, has := p.ids[pid]
if !has {
return
}
delete(p.ids, pid)
pd.ch <- err
}
type node struct {
*conn.Node
server *Server
ctx context.Context
props proposals
subscribers map[uint32]chan struct{}
}
func (n *node) RegisterForUpdates(ch chan struct{}) uint32 {
n.Lock()
defer n.Unlock()
if n.subscribers == nil {
n.subscribers = make(map[uint32]chan struct{})
}
for {
id := rand.Uint32()
if _, has := n.subscribers[id]; has {
continue
}
n.subscribers[id] = ch
return id
}
}
func (n *node) Deregister(id uint32) {
n.Lock()
defer n.Unlock()
delete(n.subscribers, id)
}
func (n *node) triggerUpdates() {
n.Lock()
defer n.Unlock()
for _, ch := range n.subscribers {
select {
case ch <- struct{}{}:
// We can ignore it and don't send a notification, because they are going to
// read a state version after now since ch is already full.
default:
}
}
}
func (n *node) AmLeader() bool {
if n.Raft() == nil {
return false
}
r := n.Raft()
return r.Status().Lead == r.Status().ID
}
func (n *node) proposeAndWait(ctx context.Context, proposal *protos.ZeroProposal) error {
if n.Raft() == nil {
return x.Errorf("Raft isn't initialized yet.")
}
if ctx.Err() != nil {
return ctx.Err()
}
che := make(chan error, 1)
pctx := &proposalCtx{
ch: che,
ctx: ctx,
}
for {
id := rand.Uint32() + 1
if n.props.Store(id, pctx) {
proposal.Id = id
break
}
}
data, err := proposal.Marshal()
if err != nil {
return err
}
// Propose the change.
if err := n.Raft().Propose(ctx, data); err != nil {
return x.Wrapf(err, "While proposing")
}
// Wait for proposal to be applied or timeout.
select {
case err := <-che:
return err
case <-ctx.Done():
return ctx.Err()
}
}
var (
errInvalidProposal = errors.New("Invalid group proposal")
errTabletAlreadyServed = errors.New("Tablet is already being served")
)
func newGroup() *protos.Group {
return &protos.Group{
Members: make(map[uint64]*protos.Member),
Tablets: make(map[string]*protos.Tablet),
}
}
func (n *node) applyProposal(e raftpb.Entry) (uint32, error) {
var p protos.ZeroProposal
// Raft commits empty entry on becoming a leader.
if len(e.Data) == 0 {
return p.Id, nil
}
if err := p.Unmarshal(e.Data); err != nil {
return p.Id, err
}
if p.Id == 0 {
return 0, errInvalidProposal
}
n.server.Lock()
defer n.server.Unlock()
state := n.server.state
state.Counter = e.Index
if p.Member != nil {
if p.Member.GroupId == 0 {
state.Zeros[p.Member.Id] = p.Member
return p.Id, nil
}
group := state.Groups[p.Member.GroupId]
if group == nil {
group = newGroup()
state.Groups[p.Member.GroupId] = group
}
_, has := group.Members[p.Member.Id]
if !has && len(group.Members) >= n.server.NumReplicas {
// We shouldn't allow more members than the number of replicas.
return p.Id, errInvalidProposal
}
group.Members[p.Member.Id] = p.Member
// On replay of logs on restart we need to set nextGroup.
if n.server.nextGroup <= p.Member.GroupId {
n.server.nextGroup = p.Member.GroupId + 1
}
}
if p.Tablet != nil {
if p.Tablet.GroupId == 0 {
return p.Id, errInvalidProposal
}
group := state.Groups[p.Tablet.GroupId]
if group == nil {
group = newGroup()
state.Groups[p.Tablet.GroupId] = group
}
// There's a edge case that we're handling.
// Two servers ask to serve the same tablet, then we need to ensure that
// only the first one succeeds.
if tablet := n.server.servingTablet(p.Tablet.Predicate); tablet != nil {
if p.Tablet.Force {
group := state.Groups[tablet.GroupId]
delete(group.Tablets, p.Tablet.Predicate)
} else {
if tablet.GroupId != p.Tablet.GroupId {
return p.Id, errTabletAlreadyServed
}
// This update can come from tablet size.
p.Tablet.ReadOnly = tablet.ReadOnly
}
}
group.Tablets[p.Tablet.Predicate] = p.Tablet
}
if p.MaxLeaseId > state.MaxLeaseId {
state.MaxLeaseId = p.MaxLeaseId
} else if p.MaxLeaseId != 0 {
x.Printf("Could not apply lease, ignoring: proposedLease=%d existingLease=%d",
p.MaxLeaseId, state.MaxLeaseId)
}
if p.MaxLeaseId == 0 {
// Don't show lease proposals - they occur too frequently to be useful.
x.Printf("Applied proposal: %+v\n", p)
}
return p.Id, nil
}
func (n *node) applyConfChange(e raftpb.Entry) {
var cc raftpb.ConfChange
cc.Unmarshal(e.Data)
if len(cc.Context) > 0 {
var rc protos.RaftContext
x.Check(rc.Unmarshal(cc.Context))
n.Connect(rc.Id, rc.Addr)
m := &protos.Member{Id: rc.Id, Addr: rc.Addr, GroupId: 0}
n.server.storeZero(m)
}
cs := n.Raft().ApplyConfChange(cc)
n.SetConfState(cs)
n.triggerLeaderChange()
}
func (n *node) triggerLeaderChange() {
select {
case n.server.leaderChangeCh <- struct{}{}:
default:
// Ok to ingore
}
m := &protos.Member{Id: n.Id, Addr: n.RaftContext.Addr, Leader: n.AmLeader()}
go n.proposeAndWait(context.Background(), &protos.ZeroProposal{Member: m})
}
func (n *node) initAndStartNode(wal *raftwal.Wal) error {
idx, restart, err := n.InitFromWal(wal)
n.Applied.SetDoneUntil(idx)
x.Check(err)
if restart {
x.Println("Restarting node for dgraphzero")
sp, err := n.Store.Snapshot()
var state protos.MembershipState
x.Check(state.Unmarshal(sp.Data))
n.server.SetMembershipState(&state)
x.Checkf(err, "Unable to get existing snapshot")
n.SetRaft(raft.RestartNode(n.Cfg))
} else if len(*peer) > 0 {
p := conn.Get().Connect(*peer)
if p == nil {
return errInvalidAddress
}
gconn := p.Get()
c := protos.NewRaftClient(gconn)
err = errJoinCluster
for err != nil {
time.Sleep(time.Millisecond)
_, err = c.JoinCluster(n.ctx, n.RaftContext)
if grpc.ErrorDesc(err) == conn.ErrDuplicateRaftId.Error() {
x.Fatalf("Error while joining cluster %v", err)
}
x.Printf("Error while joining cluster %v\n", err)
}
n.SetRaft(raft.StartNode(n.Cfg, nil))
} else {
data, err := n.RaftContext.Marshal()
x.Check(err)
peers := []raft.Peer{{ID: n.Id, Context: data}}
n.SetRaft(raft.StartNode(n.Cfg, peers))
}
go n.Run()
go n.BatchAndSendMessages()
return err
}
func (n *node) trySnapshot() {
existing, err := n.Store.Snapshot()
x.Checkf(err, "Unable to get existing snapshot")
si := existing.Metadata.Index
idx := n.Applied.DoneUntil()
if idx <= si+1000 {
return
}
data, err := n.server.MarshalMembershipState()
x.Check(err)
if tr, ok := trace.FromContext(n.ctx); ok {
tr.LazyPrintf("Taking snapshot of state at watermark: %d\n", idx)
}
s, err := n.Store.CreateSnapshot(idx, n.ConfState(), data)
x.Checkf(err, "While creating snapshot")
x.Checkf(n.Store.Compact(idx), "While compacting snapshot")
x.Check(n.Wal.StoreSnapshot(0, s))
}
func (n *node) Run() {
var leader bool
ticker := time.NewTicker(20 * time.Millisecond)
defer ticker.Stop()
rcBytes, err := n.RaftContext.Marshal()
x.Check(err)
// This chan could have capacity zero, because runReadIndexLoop never blocks without selecting
// on readStateCh. It's 2 so that sending rarely blocks (so the Go runtime doesn't have to
// switch threads as much.)
readStateCh := make(chan raft.ReadState, 2)
closer := y.NewCloser(1)
// We only stop runReadIndexLoop after the for loop below has finished interacting with it.
// That way we know sending to readStateCh will not deadlock.
defer closer.SignalAndWait()
go n.RunReadIndexLoop(closer, readStateCh)
for {
select {
case <-ticker.C:
n.Raft().Tick()
case rd := <-n.Raft().Ready():
for _, rs := range rd.ReadStates {
readStateCh <- rs
}
// First store the entries, then the hardstate and snapshot.
x.Check(n.Wal.Store(0, rd.HardState, rd.Entries))
x.Check(n.Wal.StoreSnapshot(0, rd.Snapshot))
// Now store them in the in-memory store.
n.SaveToStorage(rd.Snapshot, rd.HardState, rd.Entries)
if !raft.IsEmptySnap(rd.Snapshot) {
var state protos.MembershipState
x.Check(state.Unmarshal(rd.Snapshot.Data))
n.server.SetMembershipState(&state)
}
for _, entry := range rd.CommittedEntries {
n.Applied.Begin(entry.Index)
if entry.Type == raftpb.EntryConfChange {
n.applyConfChange(entry)
} else if entry.Type == raftpb.EntryNormal {
pid, err := n.applyProposal(entry)
if err != nil {
x.Printf("While applying proposal: %v\n", err)
}
n.props.Done(pid, err)
} else {
x.Printf("Unhandled entry: %+v\n", entry)
}
n.Applied.Done(entry.Index)
}
// TODO: Should we move this to the top?
if rd.SoftState != nil {
if rd.RaftState == raft.StateLeader && !leader {
n.server.updateNextLeaseId()
leader = true
}
n.triggerLeaderChange()
}
for _, msg := range rd.Messages {
msg.Context = rcBytes
n.Send(msg)
}
if len(rd.CommittedEntries) > 0 {
n.triggerUpdates()
}
n.trySnapshot()
n.Raft().Advance()
}
}
}