@@ -60,7 +60,7 @@ func NodeToString(role NodeRole) string {
60
60
// raft stack definition
61
61
type Raft struct {
62
62
mu sync.RWMutex
63
- peers []* RaftClientEnd // rpc client end
63
+ peers []* RaftPeerNode // rpc client end
64
64
id int64
65
65
dead int32
66
66
applyCh chan * pb.ApplyMsg
@@ -71,7 +71,6 @@ type Raft struct {
71
71
votedFor int64
72
72
grantedVotes int
73
73
logs * RaftLog
74
- persister * RaftLog
75
74
commitIdx int64
76
75
lastApplied int64
77
76
nextIdx []int
@@ -85,7 +84,7 @@ type Raft struct {
85
84
baseElecTimeout uint64
86
85
}
87
86
88
- func MakeRaft (peers []* RaftClientEnd , me int64 , newdbEng storage_eng.KvStore , applyCh chan * pb.ApplyMsg , heartbeatTimeOutMs uint64 , baseElectionTimeOutMs uint64 ) * Raft {
87
+ func MakeRaft (peers []* RaftPeerNode , me int64 , newdbEng storage_eng.KvStore , applyCh chan * pb.ApplyMsg , heartbeatTimeOutMs uint64 , baseElectionTimeOutMs uint64 ) * Raft {
89
88
rf := & Raft {
90
89
peers : peers ,
91
90
id : me ,
@@ -98,15 +97,14 @@ func MakeRaft(peers []*RaftClientEnd, me int64, newdbEng storage_eng.KvStore, ap
98
97
grantedVotes : 0 ,
99
98
isSnapshoting : false ,
100
99
logs : MakePersistRaftLog (newdbEng ),
101
- persister : MakePersistRaftLog (newdbEng ),
102
100
nextIdx : make ([]int , len (peers )),
103
101
matchIdx : make ([]int , len (peers )),
104
102
heartbeatTimer : time .NewTimer (time .Millisecond * time .Duration (heartbeatTimeOutMs )),
105
103
electionTimer : time .NewTimer (time .Millisecond * time .Duration (MakeAnRandomElectionTimeout (int (baseElectionTimeOutMs )))),
106
104
baseElecTimeout : baseElectionTimeOutMs ,
107
105
heartBeatTimeout : heartbeatTimeOutMs ,
108
106
}
109
- rf .curTerm , rf .votedFor = rf .persister .ReadRaftState ()
107
+ rf .curTerm , rf .votedFor = rf .logs .ReadRaftState ()
110
108
rf .applyCond = sync .NewCond (& rf .mu )
111
109
last_log := rf .logs .GetLast ()
112
110
for _ , peer := range peers {
@@ -126,7 +124,7 @@ func MakeRaft(peers []*RaftClientEnd, me int64, newdbEng storage_eng.KvStore, ap
126
124
}
127
125
128
126
func (rf * Raft ) PersistRaftState () {
129
- rf .persister .PersistRaftState (rf .curTerm , rf .votedFor )
127
+ rf .logs .PersistRaftState (rf .curTerm , rf .votedFor )
130
128
}
131
129
132
130
func (rf * Raft ) Kill () {
@@ -137,10 +135,6 @@ func (rf *Raft) IsKilled() bool {
137
135
return atomic .LoadInt32 (& rf .dead ) == 1
138
136
}
139
137
140
- func (rf * Raft ) GetFirstLogEnt () * pb.Entry {
141
- return rf .logs .GetFirst ()
142
- }
143
-
144
138
func (rf * Raft ) SwitchRaftNodeRole (role NodeRole ) {
145
139
if rf .role == role {
146
140
return
@@ -257,9 +251,9 @@ func (rf *Raft) HandleAppendEntries(req *pb.AppendEntriesRequest, resp *pb.Appen
257
251
resp .ConflictIndex = last_index + 1
258
252
} else {
259
253
first_index := rf .logs .GetFirst ().Index
260
- resp .ConflictTerm = int64 (rf .logs .GetEntry (req .PrevLogIndex - int64 ( first_index ) ).Term )
254
+ resp .ConflictTerm = int64 (rf .logs .GetEntry (req .PrevLogIndex ).Term )
261
255
index := req .PrevLogIndex - 1
262
- for index >= int64 (first_index ) && rf .logs .GetEntry (index - first_index ).Term == uint64 (resp .ConflictTerm ) {
256
+ for index >= int64 (first_index ) && rf .logs .GetEntry (index ).Term == uint64 (resp .ConflictTerm ) {
263
257
index --
264
258
}
265
259
resp .ConflictIndex = index
@@ -269,7 +263,7 @@ func (rf *Raft) HandleAppendEntries(req *pb.AppendEntriesRequest, resp *pb.Appen
269
263
270
264
first_index := rf .logs .GetFirst ().Index
271
265
for index , entry := range req .Entries {
272
- if int (entry .Index - first_index ) >= rf .logs .LogItemCount () || rf .logs .GetEntry (entry .Index - first_index ).Term != entry .Term {
266
+ if int (entry .Index - first_index ) >= rf .logs .LogItemCount () || rf .logs .GetEntry (entry .Index ).Term != entry .Term {
273
267
rf .logs .EraseAfter (entry .Index - first_index , true )
274
268
for _ , newEnt := range req .Entries [index :] {
275
269
rf .logs .Append (newEnt )
@@ -294,11 +288,11 @@ func (rf *Raft) CondInstallSnapshot(lastIncluedTerm int, lastIncludedIndex int,
294
288
if lastIncludedIndex > int (rf .logs .GetLast ().Index ) {
295
289
rf .logs .ReInitLogs ()
296
290
} else {
297
- rf .logs .EraseBefore (int64 (lastIncludedIndex )- rf . logs . GetFirst (). Index , true )
291
+ rf .logs .EraseBefore (int64 (lastIncludedIndex ), true )
298
292
rf .logs .SetEntFirstData ([]byte {})
299
293
}
300
294
// update dummy entry with lastIncludedTerm and lastIncludedIndex
301
- rf .logs .SetEntFirstTermAndIndex (int64 (lastIncluedTerm ), int64 (lastIncludedIndex ))
295
+ rf .logs .ResetFirstEntryTermAndIndex (int64 (lastIncluedTerm ), int64 (lastIncludedIndex ))
302
296
303
297
rf .lastApplied = int64 (lastIncludedIndex )
304
298
rf .commitIdx = int64 (lastIncludedIndex )
@@ -317,7 +311,7 @@ func (rf *Raft) Snapshot(index int, snapshot []byte) {
317
311
logger .ELogger ().Sugar ().Warnf ("reject snapshot, current snapshotIndex is larger in cur term" )
318
312
return
319
313
}
320
- rf .logs .EraseBefore (int64 (index )- int64 ( snapshot_index ) , true )
314
+ rf .logs .EraseBefore (int64 (index ), true )
321
315
rf .logs .SetEntFirstData ([]byte {})
322
316
logger .ELogger ().Sugar ().Debugf ("del log entry before idx %d" , index )
323
317
rf .isSnapshoting = false
@@ -382,7 +376,7 @@ func (rf *Raft) advanceCommitIndexForLeader() {
382
376
new_commit_index := rf .matchIdx [n - (n / 2 + 1 )]
383
377
if new_commit_index > int (rf .commitIdx ) {
384
378
if rf .MatchLog (rf .curTerm , int64 (new_commit_index )) {
385
- logger .ELogger ().Sugar ().Debugf ("peer %d advance commit index %d at term %d" , rf .id , rf .commitIdx , rf .curTerm )
379
+ logger .ELogger ().Sugar ().Debugf ("leader advance commit lid %d index %d at term %d" , rf .id , rf .commitIdx , rf .curTerm )
386
380
rf .commitIdx = int64 (new_commit_index )
387
381
rf .applyCond .Signal ()
388
382
}
@@ -400,7 +394,7 @@ func (rf *Raft) advanceCommitIndexForFollower(leaderCommit int) {
400
394
401
395
// MatchLog is log matched
402
396
func (rf * Raft ) MatchLog (term , index int64 ) bool {
403
- return index <= int64 (rf .logs .GetLast ().Index ) && rf .logs .GetEntry (index - int64 ( rf . logs . GetFirst (). Index ) ).Term == uint64 (term )
397
+ return index <= int64 (rf .logs .GetLast ().Index ) && rf .logs .GetEntry (index ).Term == uint64 (term )
404
398
}
405
399
406
400
// Election make a new election
@@ -420,7 +414,7 @@ func (rf *Raft) Election() {
420
414
if int64 (peer .id ) == rf .id {
421
415
continue
422
416
}
423
- go func (peer * RaftClientEnd ) {
417
+ go func (peer * RaftPeerNode ) {
424
418
logger .ELogger ().Sugar ().Debugf ("send request vote to %s %s" , peer .addr , vote_req .String ())
425
419
426
420
request_vote_resp , err := (* peer .raftServiceCli ).RequestVote (context .Background (), vote_req )
@@ -473,7 +467,7 @@ func (rf *Raft) BroadcastHeartbeat() {
473
467
continue
474
468
}
475
469
logger .ELogger ().Sugar ().Debugf ("send heart beat to %s" , peer .addr )
476
- go func (peer * RaftClientEnd ) {
470
+ go func (peer * RaftPeerNode ) {
477
471
rf .replicateOneRound (peer )
478
472
}(peer )
479
473
}
@@ -545,7 +539,7 @@ func (rf *Raft) CloseEndsConn() {
545
539
}
546
540
547
541
// Replicator manager duplicate run
548
- func (rf * Raft ) Replicator (peer * RaftClientEnd ) {
542
+ func (rf * Raft ) Replicator (peer * RaftPeerNode ) {
549
543
rf .replicatorCond [peer .id ].L .Lock ()
550
544
defer rf .replicatorCond [peer .id ].L .Unlock ()
551
545
for ! rf .IsKilled () {
@@ -558,7 +552,7 @@ func (rf *Raft) Replicator(peer *RaftClientEnd) {
558
552
}
559
553
560
554
// replicateOneRound duplicate log entries to other nodes in the cluster
561
- func (rf * Raft ) replicateOneRound (peer * RaftClientEnd ) {
555
+ func (rf * Raft ) replicateOneRound (peer * RaftPeerNode ) {
562
556
rf .mu .RLock ()
563
557
if rf .role != NodeRoleLeader {
564
558
rf .mu .RUnlock ()
@@ -606,15 +600,15 @@ func (rf *Raft) replicateOneRound(peer *RaftClientEnd) {
606
600
} else {
607
601
first_index := rf .logs .GetFirst ().Index
608
602
logger .ELogger ().Sugar ().Debugf ("first log index %d" , first_index )
609
- _ , new_ents := rf .logs .EraseBefore (int64 (prev_log_index )+ 1 - first_index , false )
603
+ new_ents , _ := rf .logs .EraseBefore (int64 (prev_log_index )+ 1 , false )
610
604
entries := make ([]* pb.Entry , len (new_ents ))
611
605
copy (entries , new_ents )
612
606
613
607
append_ent_req := & pb.AppendEntriesRequest {
614
608
Term : rf .curTerm ,
615
609
LeaderId : int64 (rf .id ),
616
610
PrevLogIndex : int64 (prev_log_index ),
617
- PrevLogTerm : int64 (rf .logs .GetEntry (int64 (prev_log_index ) - first_index ).Term ),
611
+ PrevLogTerm : int64 (rf .logs .GetEntry (int64 (prev_log_index )).Term ),
618
612
Entries : entries ,
619
613
LeaderCommit : rf .commitIdx ,
620
614
}
@@ -644,7 +638,7 @@ func (rf *Raft) replicateOneRound(peer *RaftClientEnd) {
644
638
rf .nextIdx [peer .id ] = int (resp .ConflictIndex )
645
639
if resp .ConflictTerm != - 1 {
646
640
for i := append_ent_req .PrevLogIndex ; i >= int64 (first_index ); i -- {
647
- if rf .logs .GetEntry (i - int64 ( first_index ) ).Term == uint64 (resp .ConflictTerm ) {
641
+ if rf .logs .GetEntry (i ).Term == uint64 (resp .ConflictTerm ) {
648
642
rf .nextIdx [peer .id ] = int (i + 1 )
649
643
break
650
644
}
@@ -667,9 +661,9 @@ func (rf *Raft) Applier() {
667
661
rf .applyCond .Wait ()
668
662
}
669
663
670
- first_index , commit_index , last_applied := rf . logs . GetFirst (). Index , rf .commitIdx , rf .lastApplied
664
+ commit_index , last_applied := rf .commitIdx , rf .lastApplied
671
665
entries := make ([]* pb.Entry , commit_index - last_applied )
672
- copy (entries , rf .logs .GetRange (last_applied + 1 - int64 ( first_index ) , commit_index + 1 - int64 ( first_index ) ))
666
+ copy (entries , rf .logs .GetRange (last_applied + 1 , commit_index ))
673
667
logger .ELogger ().Sugar ().Debugf ("%d, applies entries %d-%d in term %d" , rf .id , rf .lastApplied , commit_index , rf .curTerm )
674
668
675
669
rf .mu .Unlock ()
0 commit comments