forked from hashicorp/raft
-
Notifications
You must be signed in to change notification settings - Fork 0
/
Copy pathinflight.go
176 lines (146 loc) · 3.92 KB
/
inflight.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
package raft
import (
"net"
"sync"
)
// QuorumPolicy allows individual logFutures to have different
// commitment rules while still using the inflight mechanism
type quorumPolicy interface {
// Checks if a commit from a given peer is enough to
// satisfy the commitment rules
Commit(net.Addr) bool
// Checks if a commit is committed
IsCommitted() bool
}
// MajorityQuorum is used by Apply transactions and requires
// a simple majority of nodes
type majorityQuorum struct {
count int
votesNeeded int
}
func newMajorityQuorum(clusterSize int) *majorityQuorum {
votesNeeded := (clusterSize / 2) + 1
return &majorityQuorum{count: 0, votesNeeded: votesNeeded}
}
func (m *majorityQuorum) Commit(p net.Addr) bool {
m.count++
return m.count >= m.votesNeeded
}
func (m *majorityQuorum) IsCommitted() bool {
return m.count >= m.votesNeeded
}
// Inflight is used to track operations that are still in-flight
type inflight struct {
sync.Mutex
commitCh chan *logFuture
minCommit uint64
maxCommit uint64
operations map[uint64]*logFuture
stopCh chan struct{}
}
// NewInflight returns an inflight struct that notifies
// the provided channel when logs are finished commiting.
func newInflight(commitCh chan *logFuture) *inflight {
return &inflight{
commitCh: commitCh,
minCommit: 0,
maxCommit: 0,
operations: make(map[uint64]*logFuture),
stopCh: make(chan struct{}),
}
}
// Start is used to mark a logFuture as being inflight
func (i *inflight) Start(l *logFuture) {
i.Lock()
defer i.Unlock()
idx := l.log.Index
i.operations[idx] = l
if idx > i.maxCommit {
i.maxCommit = idx
}
if i.minCommit == 0 {
i.minCommit = idx
}
}
// Cancel is used to cancel all in-flight operations.
// This is done when the leader steps down, and all futures
// are sent the given error.
func (i *inflight) Cancel(err error) {
// Close the channel first to unblock any pending commits
close(i.stopCh)
// Lock after close to avoid deadlock
i.Lock()
defer i.Unlock()
// Respond to all inflight operations
for _, op := range i.operations {
op.respond(err)
}
// Clear the map
i.operations = make(map[uint64]*logFuture)
// Close the commmitCh
close(i.commitCh)
// Reset indexes
i.minCommit = 0
i.maxCommit = 0
}
// Commit is used by leader replication routines to indicate that
// a follower was finished commiting a log to disk.
func (i *inflight) Commit(index uint64, peer net.Addr) {
i.Lock()
defer i.Unlock()
i.commit(index, peer)
}
// CommitRange is used to commit a range of indexes inclusively
// It optimized to avoid commits for indexes that are not tracked
func (i *inflight) CommitRange(minIndex, maxIndex uint64, peer net.Addr) {
i.Lock()
defer i.Unlock()
// Update the minimum index
minIndex = max(i.minCommit, minIndex)
// Commit each index
for idx := minIndex; idx <= maxIndex; idx++ {
i.commit(idx, peer)
}
}
// commit is used to commit a single index. Must be called with the lock held.
func (i *inflight) commit(index uint64, peer net.Addr) {
op, ok := i.operations[index]
if !ok {
// Ignore if not in the map, as it may be commited already
return
}
// Check if we've satisfied the commit
if !op.policy.Commit(peer) {
return
}
// Cannot commit if this is not the minimum inflight. This can happen
// if the quorum size changes, meaning a previous commit requires a larger
// quorum that this commit. We MUST block until the previous log is committed,
// otherwise logs will be applied out of order.
if index != i.minCommit {
return
}
// Notify of commit
NOTIFY:
select {
case i.commitCh <- op:
// Stop tracking since it is committed
delete(i.operations, index)
// Update the indexes
if index == i.maxCommit {
i.minCommit = 0
i.maxCommit = 0
} else {
i.minCommit++
}
case <-i.stopCh:
}
// Check if the next in-flight operation is ready
if i.minCommit != 0 {
op = i.operations[i.minCommit]
if op.policy.IsCommitted() {
index = i.minCommit
goto NOTIFY
}
}
}