Browse Source

raft: log->raftlog

Xiang Li 11 years ago
parent
commit
1288e1f39d
8 changed files with 178 additions and 178 deletions
  1. 2 2
      raft/cluster_test.go
  2. 1 1
      raft/diff_test.go
  3. 20 20
      raft/log.go
  4. 40 40
      raft/log_test.go
  5. 5 5
      raft/node.go
  6. 4 4
      raft/node_test.go
  7. 40 40
      raft/raft.go
  8. 66 66
      raft/raft_test.go

+ 2 - 2
raft/cluster_test.go

@@ -26,10 +26,10 @@ func TestBuildCluster(t *testing.T) {
 	for i, tt := range tests {
 		_, nodes := buildCluster(tt.size, tt.ids)
 
-		base := ltoa(nodes[0].sm.log)
+		base := ltoa(nodes[0].sm.raftLog)
 		for j, n := range nodes {
 			// ensure same log
-			l := ltoa(n.sm.log)
+			l := ltoa(n.sm.raftLog)
 			if g := diffu(base, l); g != "" {
 				t.Errorf("#%d.%d: log diff:\n%s", i, j, g)
 			}

+ 1 - 1
raft/diff_test.go

@@ -41,7 +41,7 @@ func mustTemp(pre, body string) string {
 	return f.Name()
 }
 
-func ltoa(l *log) string {
+func ltoa(l *raftLog) string {
 	s := fmt.Sprintf("committed: %d\n", l.committed)
 	s += fmt.Sprintf("applied:  %d\n", l.applied)
 	for i, e := range l.ents {

+ 20 - 20
raft/log.go

@@ -24,7 +24,7 @@ func (e *Entry) isConfig() bool {
 	return e.Type == AddNode || e.Type == RemoveNode
 }
 
-type log struct {
+type raftLog struct {
 	ents      []Entry
 	committed int64
 	applied   int64
@@ -35,8 +35,8 @@ type log struct {
 	compactThreshold int64
 }
 
-func newLog() *log {
-	return &log{
+func newLog() *raftLog {
+	return &raftLog{
 		ents:             make([]Entry, 1),
 		committed:        0,
 		applied:          0,
@@ -44,11 +44,11 @@ func newLog() *log {
 	}
 }
 
-func (l *log) String() string {
+func (l *raftLog) String() string {
 	return fmt.Sprintf("offset=%d committed=%d applied=%d len(ents)=%d", l.offset, l.committed, l.applied, len(l.ents))
 }
 
-func (l *log) maybeAppend(index, logTerm, committed int64, ents ...Entry) bool {
+func (l *raftLog) maybeAppend(index, logTerm, committed int64, ents ...Entry) bool {
 	if l.matchTerm(index, logTerm) {
 		from := index + 1
 		ci := l.findConflict(from, ents)
@@ -67,12 +67,12 @@ func (l *log) maybeAppend(index, logTerm, committed int64, ents ...Entry) bool {
 	return false
 }
 
-func (l *log) append(after int64, ents ...Entry) int64 {
+func (l *raftLog) append(after int64, ents ...Entry) int64 {
 	l.ents = append(l.slice(l.offset, after+1), ents...)
 	return l.lastIndex()
 }
 
-func (l *log) findConflict(from int64, ents []Entry) int64 {
+func (l *raftLog) findConflict(from int64, ents []Entry) int64 {
 	for i, ne := range ents {
 		if oe := l.at(from + int64(i)); oe == nil || oe.Term != ne.Term {
 			return from + int64(i)
@@ -81,18 +81,18 @@ func (l *log) findConflict(from int64, ents []Entry) int64 {
 	return -1
 }
 
-func (l *log) lastIndex() int64 {
+func (l *raftLog) lastIndex() int64 {
 	return int64(len(l.ents)) - 1 + l.offset
 }
 
-func (l *log) term(i int64) int64 {
+func (l *raftLog) term(i int64) int64 {
 	if e := l.at(i); e != nil {
 		return e.Term
 	}
 	return -1
 }
 
-func (l *log) entries(i int64) []Entry {
+func (l *raftLog) entries(i int64) []Entry {
 	// never send out the first entry
 	// first entry is only used for matching
 	// prevLogTerm
@@ -102,19 +102,19 @@ func (l *log) entries(i int64) []Entry {
 	return l.slice(i, l.lastIndex()+1)
 }
 
-func (l *log) isUpToDate(i, term int64) bool {
+func (l *raftLog) isUpToDate(i, term int64) bool {
 	e := l.at(l.lastIndex())
 	return term > e.Term || (term == e.Term && i >= l.lastIndex())
 }
 
-func (l *log) matchTerm(i, term int64) bool {
+func (l *raftLog) matchTerm(i, term int64) bool {
 	if e := l.at(i); e != nil {
 		return e.Term == term
 	}
 	return false
 }
 
-func (l *log) maybeCommit(maxIndex, term int64) bool {
+func (l *raftLog) maybeCommit(maxIndex, term int64) bool {
 	if maxIndex > l.committed && l.term(maxIndex) == term {
 		l.committed = maxIndex
 		return true
@@ -124,7 +124,7 @@ func (l *log) maybeCommit(maxIndex, term int64) bool {
 
 // nextEnts returns all the available entries for execution.
 // all the returned entries will be marked as applied.
-func (l *log) nextEnts() (ents []Entry) {
+func (l *raftLog) nextEnts() (ents []Entry) {
 	if l.committed > l.applied {
 		ents = l.slice(l.applied+1, l.committed+1)
 		l.applied = l.committed
@@ -136,7 +136,7 @@ func (l *log) nextEnts() (ents []Entry) {
 // i must be not smaller than the index of the first entry
 // and not greater than the index of the last entry.
 // the number of entries after compaction will be returned.
-func (l *log) compact(i int64) int64 {
+func (l *raftLog) compact(i int64) int64 {
 	if l.isOutOfBounds(i) {
 		panic(fmt.Sprintf("compact %d out of bounds [%d:%d]", i, l.offset, l.lastIndex()))
 	}
@@ -145,18 +145,18 @@ func (l *log) compact(i int64) int64 {
 	return int64(len(l.ents))
 }
 
-func (l *log) shouldCompact() bool {
+func (l *raftLog) shouldCompact() bool {
 	return (l.applied - l.offset) > l.compactThreshold
 }
 
-func (l *log) restore(index, term int64) {
+func (l *raftLog) restore(index, term int64) {
 	l.ents = []Entry{{Term: term}}
 	l.committed = index
 	l.applied = index
 	l.offset = index
 }
 
-func (l *log) at(i int64) *Entry {
+func (l *raftLog) at(i int64) *Entry {
 	if l.isOutOfBounds(i) {
 		return nil
 	}
@@ -164,7 +164,7 @@ func (l *log) at(i int64) *Entry {
 }
 
 // slice get a slice of log entries from lo through hi-1, inclusive.
-func (l *log) slice(lo int64, hi int64) []Entry {
+func (l *raftLog) slice(lo int64, hi int64) []Entry {
 	if lo >= hi {
 		return nil
 	}
@@ -174,7 +174,7 @@ func (l *log) slice(lo int64, hi int64) []Entry {
 	return l.ents[lo-l.offset : hi-l.offset]
 }
 
-func (l *log) isOutOfBounds(i int64) bool {
+func (l *raftLog) isOutOfBounds(i int64) bool {
 	if i < l.offset || i > l.lastIndex() {
 		return true
 	}

+ 40 - 40
raft/log_test.go

@@ -47,13 +47,13 @@ func TestAppend(t *testing.T) {
 	}
 
 	for i, tt := range tests {
-		log := newLog()
-		log.ents = append(log.ents, previousEnts...)
-		index := log.append(tt.after, tt.ents...)
+		raftLog := newLog()
+		raftLog.ents = append(raftLog.ents, previousEnts...)
+		index := raftLog.append(tt.after, tt.ents...)
 		if index != tt.windex {
 			t.Errorf("#%d: lastIndex = %d, want %d", i, index, tt.windex)
 		}
-		if g := log.entries(1); !reflect.DeepEqual(g, tt.wents) {
+		if g := raftLog.entries(1); !reflect.DeepEqual(g, tt.wents) {
 			t.Errorf("#%d: logEnts = %+v, want %+v", i, g, tt.wents)
 		}
 	}
@@ -64,37 +64,37 @@ func TestAppend(t *testing.T) {
 func TestCompactionSideEffects(t *testing.T) {
 	var i int64
 	lastIndex := int64(1000)
-	log := newLog()
+	raftLog := newLog()
 
 	for i = 0; i < lastIndex; i++ {
-		log.append(int64(i), Entry{Term: int64(i + 1)})
+		raftLog.append(int64(i), Entry{Term: int64(i + 1)})
 	}
 
-	log.compact(500)
+	raftLog.compact(500)
 
-	if log.lastIndex() != lastIndex {
-		t.Errorf("lastIndex = %d, want %d", log.lastIndex(), lastIndex)
+	if raftLog.lastIndex() != lastIndex {
+		t.Errorf("lastIndex = %d, want %d", raftLog.lastIndex(), lastIndex)
 	}
 
-	for i := log.offset; i <= log.lastIndex(); i++ {
-		if log.term(i) != i {
-			t.Errorf("term(%d) = %d, want %d", i, log.term(i), i)
+	for i := raftLog.offset; i <= raftLog.lastIndex(); i++ {
+		if raftLog.term(i) != i {
+			t.Errorf("term(%d) = %d, want %d", i, raftLog.term(i), i)
 		}
 	}
 
-	for i := log.offset; i <= log.lastIndex(); i++ {
-		if !log.matchTerm(i, i) {
+	for i := raftLog.offset; i <= raftLog.lastIndex(); i++ {
+		if !raftLog.matchTerm(i, i) {
 			t.Errorf("matchTerm(%d) = false, want true", i)
 		}
 	}
 
-	prev := log.lastIndex()
-	log.append(log.lastIndex(), Entry{Term: log.lastIndex() + 1})
-	if log.lastIndex() != prev+1 {
-		t.Errorf("lastIndex = %d, want = %d", log.lastIndex(), prev+1)
+	prev := raftLog.lastIndex()
+	raftLog.append(raftLog.lastIndex(), Entry{Term: raftLog.lastIndex() + 1})
+	if raftLog.lastIndex() != prev+1 {
+		t.Errorf("lastIndex = %d, want = %d", raftLog.lastIndex(), prev+1)
 	}
 
-	ents := log.entries(log.lastIndex())
+	ents := raftLog.entries(raftLog.lastIndex())
 	if len(ents) != 1 {
 		t.Errorf("len(entries) = %d, want = %d", len(ents), 1)
 	}
@@ -125,15 +125,15 @@ func TestCompaction(t *testing.T) {
 				}
 			}()
 
-			log := newLog()
+			raftLog := newLog()
 			for i := 0; i < tt.app; i++ {
-				log.append(int64(i), Entry{})
+				raftLog.append(int64(i), Entry{})
 			}
 
 			for j := 0; j < len(tt.compact); j++ {
-				log.compact(tt.compact[j])
-				if len(log.ents) != tt.wleft[j] {
-					t.Errorf("#%d.%d len = %d, want %d", i, j, len(log.ents), tt.wleft[j])
+				raftLog.compact(tt.compact[j])
+				if len(raftLog.ents) != tt.wleft[j] {
+					t.Errorf("#%d.%d len = %d, want %d", i, j, len(raftLog.ents), tt.wleft[j])
 				}
 			}
 		}()
@@ -142,37 +142,37 @@ func TestCompaction(t *testing.T) {
 
 func TestLogRestore(t *testing.T) {
 	var i int64
-	log := newLog()
+	raftLog := newLog()
 	for i = 0; i < 100; i++ {
-		log.append(i, Entry{Term: i + 1})
+		raftLog.append(i, Entry{Term: i + 1})
 	}
 
 	index := int64(1000)
 	term := int64(1000)
-	log.restore(index, term)
+	raftLog.restore(index, term)
 
 	// only has the guard entry
-	if len(log.ents) != 1 {
-		t.Errorf("len = %d, want 0", len(log.ents))
+	if len(raftLog.ents) != 1 {
+		t.Errorf("len = %d, want 0", len(raftLog.ents))
 	}
-	if log.offset != index {
-		t.Errorf("offset = %d, want %d", log.offset, index)
+	if raftLog.offset != index {
+		t.Errorf("offset = %d, want %d", raftLog.offset, index)
 	}
-	if log.applied != index {
-		t.Errorf("applied = %d, want %d", log.applied, index)
+	if raftLog.applied != index {
+		t.Errorf("applied = %d, want %d", raftLog.applied, index)
 	}
-	if log.committed != index {
-		t.Errorf("comitted = %d, want %d", log.committed, index)
+	if raftLog.committed != index {
+		t.Errorf("comitted = %d, want %d", raftLog.committed, index)
 	}
-	if log.term(index) != term {
-		t.Errorf("term = %d, want %d", log.term(index), term)
+	if raftLog.term(index) != term {
+		t.Errorf("term = %d, want %d", raftLog.term(index), term)
 	}
 }
 
 func TestIsOutOfBounds(t *testing.T) {
 	offset := int64(100)
 	num := int64(100)
-	l := &log{offset: offset, ents: make([]Entry, num)}
+	l := &raftLog{offset: offset, ents: make([]Entry, num)}
 
 	tests := []struct {
 		index int64
@@ -198,7 +198,7 @@ func TestAt(t *testing.T) {
 	offset := int64(100)
 	num := int64(100)
 
-	l := &log{offset: offset}
+	l := &raftLog{offset: offset}
 	for i = 0; i < num; i++ {
 		l.ents = append(l.ents, Entry{Term: i})
 	}
@@ -227,7 +227,7 @@ func TestSlice(t *testing.T) {
 	offset := int64(100)
 	num := int64(100)
 
-	l := &log{offset: offset}
+	l := &raftLog{offset: offset}
 	for i = 0; i < num; i++ {
 		l.ents = append(l.ents, Entry{Term: i})
 	}

+ 5 - 5
raft/node.go

@@ -3,7 +3,7 @@ package raft
 import (
 	"encoding/binary"
 	"encoding/json"
-	golog "log"
+	"log"
 	"math/rand"
 	"time"
 )
@@ -59,7 +59,7 @@ func (n *Node) Index() int64 { return n.sm.index.Get() }
 
 func (n *Node) Term() int64 { return n.sm.term.Get() }
 
-func (n *Node) Applied() int64 { return n.sm.log.applied }
+func (n *Node) Applied() int64 { return n.sm.raftLog.applied }
 
 func (n *Node) HasLeader() bool { return n.Leader() != none }
 
@@ -100,7 +100,7 @@ func (n *Node) Step(m Message) bool {
 		return false
 	}
 	if n.ClusterId() != none && m.ClusterId != none && m.ClusterId != n.ClusterId() {
-		golog.Printf("denied a message from node %d, cluster %d. accept cluster: %d\n", m.From, m.ClusterId, n.ClusterId())
+		log.Printf("denied a message from node %d, cluster %d. accept cluster: %d\n", m.From, m.ClusterId, n.ClusterId())
 		n.sm.send(Message{To: m.From, ClusterId: n.ClusterId(), Type: msgDenied})
 		return true
 	}
@@ -151,7 +151,7 @@ func (n *Node) Next() []Entry {
 		case AddNode:
 			c := new(Config)
 			if err := json.Unmarshal(ents[i].Data, c); err != nil {
-				golog.Println(err)
+				log.Println(err)
 				continue
 			}
 			n.sm.addNode(c.NodeId)
@@ -159,7 +159,7 @@ func (n *Node) Next() []Entry {
 		case RemoveNode:
 			c := new(Config)
 			if err := json.Unmarshal(ents[i].Data, c); err != nil {
-				golog.Println(err)
+				log.Println(err)
 				continue
 			}
 			n.sm.removeNode(c.NodeId)

+ 4 - 4
raft/node_test.go

@@ -80,9 +80,9 @@ func TestResetElapse(t *testing.T) {
 	for i, tt := range tests {
 		n := New(0, defaultHeartbeat, defaultElection)
 		n.sm = newStateMachine(0, []int64{0, 1, 2})
-		n.sm.log.append(0, Entry{Type: Normal, Term: 1})
+		n.sm.raftLog.append(0, Entry{Type: Normal, Term: 1})
 		n.sm.term = 2
-		n.sm.log.committed = 1
+		n.sm.raftLog.committed = 1
 
 		n.Tick()
 		if n.elapsed != 1 {
@@ -171,8 +171,8 @@ func TestDenial(t *testing.T) {
 		n := dictate(New(0, defaultHeartbeat, defaultElection))
 		n.Next()
 		n.Msgs()
-		n.sm.log.append(n.sm.log.committed, append(logents, tt.ent)...)
-		n.sm.log.committed += int64(len(logents) + 1)
+		n.sm.raftLog.append(n.sm.raftLog.committed, append(logents, tt.ent)...)
+		n.sm.raftLog.committed += int64(len(logents) + 1)
 		n.Next()
 
 		for id, denied := range tt.wdenied {

+ 40 - 40
raft/raft.go

@@ -3,7 +3,7 @@ package raft
 import (
 	"errors"
 	"fmt"
-	golog "log"
+	"log"
 	"sort"
 	"sync/atomic"
 )
@@ -134,7 +134,7 @@ type stateMachine struct {
 	vote int64
 
 	// the log
-	log *log
+	raftLog *raftLog
 
 	ins map[int64]*index
 
@@ -157,7 +157,7 @@ func newStateMachine(id int64, peers []int64) *stateMachine {
 	if id == none {
 		panic("cannot use none id")
 	}
-	sm := &stateMachine{id: id, clusterId: none, lead: none, log: newLog(), ins: make(map[int64]*index)}
+	sm := &stateMachine{id: id, clusterId: none, lead: none, raftLog: newLog(), ins: make(map[int64]*index)}
 	for _, p := range peers {
 		sm.ins[p] = &index{}
 	}
@@ -199,7 +199,7 @@ func (sm *stateMachine) send(m Message) {
 	m.ClusterId = sm.clusterId
 	m.From = sm.id
 	m.Term = sm.term.Get()
-	golog.Printf("raft.send msg %v\n", m)
+	log.Printf("raft.send msg %v\n", m)
 	sm.msgs = append(sm.msgs, m)
 }
 
@@ -214,9 +214,9 @@ func (sm *stateMachine) sendAppend(to int64) {
 		m.Snapshot = sm.snapshoter.GetSnap()
 	} else {
 		m.Type = msgApp
-		m.LogTerm = sm.log.term(in.next - 1)
-		m.Entries = sm.log.entries(in.next)
-		m.Commit = sm.log.committed
+		m.LogTerm = sm.raftLog.term(in.next - 1)
+		m.Entries = sm.raftLog.entries(in.next)
+		m.Commit = sm.raftLog.committed
 	}
 	sm.send(m)
 }
@@ -224,13 +224,13 @@ func (sm *stateMachine) sendAppend(to int64) {
 // sendHeartbeat sends RRPC, without entries to the given peer.
 func (sm *stateMachine) sendHeartbeat(to int64) {
 	in := sm.ins[to]
-	index := max(in.next-1, sm.log.lastIndex())
+	index := max(in.next-1, sm.raftLog.lastIndex())
 	m := Message{
 		To:      to,
 		Type:    msgApp,
 		Index:   index,
-		LogTerm: sm.log.term(index),
-		Commit:  sm.log.committed,
+		LogTerm: sm.raftLog.term(index),
+		Commit:  sm.raftLog.committed,
 	}
 	sm.send(m)
 }
@@ -264,12 +264,12 @@ func (sm *stateMachine) maybeCommit() bool {
 	sort.Sort(sort.Reverse(mis))
 	mci := mis[sm.q()-1]
 
-	return sm.log.maybeCommit(mci, sm.term.Get())
+	return sm.raftLog.maybeCommit(mci, sm.term.Get())
 }
 
 // nextEnts returns the appliable entries and updates the applied index
 func (sm *stateMachine) nextEnts() (ents []Entry) {
-	return sm.log.nextEnts()
+	return sm.raftLog.nextEnts()
 }
 
 func (sm *stateMachine) reset(term int64) {
@@ -278,9 +278,9 @@ func (sm *stateMachine) reset(term int64) {
 	sm.vote = none
 	sm.votes = make(map[int64]bool)
 	for i := range sm.ins {
-		sm.ins[i] = &index{next: sm.log.lastIndex() + 1}
+		sm.ins[i] = &index{next: sm.raftLog.lastIndex() + 1}
 		if i == sm.id {
-			sm.ins[i].match = sm.log.lastIndex()
+			sm.ins[i].match = sm.raftLog.lastIndex()
 		}
 	}
 }
@@ -291,8 +291,8 @@ func (sm *stateMachine) q() int {
 
 func (sm *stateMachine) appendEntry(e Entry) {
 	e.Term = sm.term.Get()
-	sm.index.Set(sm.log.append(sm.log.lastIndex(), e))
-	sm.ins[sm.id].update(sm.log.lastIndex())
+	sm.index.Set(sm.raftLog.append(sm.raftLog.lastIndex(), e))
+	sm.ins[sm.id].update(sm.raftLog.lastIndex())
 	sm.maybeCommit()
 }
 
@@ -300,7 +300,7 @@ func (sm *stateMachine) appendEntry(e Entry) {
 // New machine has to wait for the first log entry to be committed, or it will
 // always start as a one-node cluster.
 func (sm *stateMachine) promotable() bool {
-	return sm.log.committed != 0
+	return sm.raftLog.committed != 0
 }
 
 func (sm *stateMachine) becomeFollower(term int64, lead int64) {
@@ -329,7 +329,7 @@ func (sm *stateMachine) becomeLeader() {
 	sm.lead.Set(sm.id)
 	sm.state = stateLeader
 
-	for _, e := range sm.log.entries(sm.log.committed + 1) {
+	for _, e := range sm.raftLog.entries(sm.raftLog.committed + 1) {
 		if e.isConfig() {
 			sm.pendingConf = true
 		}
@@ -346,11 +346,11 @@ func (sm *stateMachine) Msgs() []Message {
 }
 
 func (sm *stateMachine) Step(m Message) (ok bool) {
-	golog.Printf("raft.step beforeState %v\n", sm)
-	golog.Printf("raft.step beforeLog %v\n", sm.log)
-	defer golog.Printf("raft.step afterLog %v\n", sm.log)
-	defer golog.Printf("raft.step afterState %v\n", sm)
-	golog.Printf("raft.step msg %v\n", m)
+	log.Printf("raft.step beforeState %v\n", sm)
+	log.Printf("raft.step beforeLog %v\n", sm.raftLog)
+	defer log.Printf("raft.step afterLog %v\n", sm.raftLog)
+	defer log.Printf("raft.step afterState %v\n", sm)
+	log.Printf("raft.step msg %v\n", m)
 	if m.Type == msgHup {
 		sm.becomeCandidate()
 		if sm.q() == sm.poll(sm.id, true) {
@@ -361,8 +361,8 @@ func (sm *stateMachine) Step(m Message) (ok bool) {
 			if i == sm.id {
 				continue
 			}
-			lasti := sm.log.lastIndex()
-			sm.send(Message{To: i, Type: msgVote, Index: lasti, LogTerm: sm.log.term(lasti)})
+			lasti := sm.raftLog.lastIndex()
+			sm.send(Message{To: i, Type: msgVote, Index: lasti, LogTerm: sm.raftLog.term(lasti)})
 		}
 		return true
 	}
@@ -385,9 +385,9 @@ func (sm *stateMachine) Step(m Message) (ok bool) {
 }
 
 func (sm *stateMachine) handleAppendEntries(m Message) {
-	if sm.log.maybeAppend(m.Index, m.LogTerm, m.Commit, m.Entries...) {
-		sm.index.Set(sm.log.lastIndex())
-		sm.send(Message{To: m.From, Type: msgAppResp, Index: sm.log.lastIndex()})
+	if sm.raftLog.maybeAppend(m.Index, m.LogTerm, m.Commit, m.Entries...) {
+		sm.index.Set(sm.raftLog.lastIndex())
+		sm.send(Message{To: m.From, Type: msgAppResp, Index: sm.raftLog.lastIndex()})
 	} else {
 		sm.send(Message{To: m.From, Type: msgAppResp, Index: -1})
 	}
@@ -395,11 +395,11 @@ func (sm *stateMachine) handleAppendEntries(m Message) {
 
 func (sm *stateMachine) handleSnapshot(m Message) {
 	sm.restore(m.Snapshot)
-	sm.send(Message{To: m.From, Type: msgAppResp, Index: sm.log.lastIndex()})
+	sm.send(Message{To: m.From, Type: msgAppResp, Index: sm.raftLog.lastIndex()})
 }
 
 func (sm *stateMachine) addNode(id int64) {
-	sm.ins[id] = &index{next: sm.log.lastIndex() + 1}
+	sm.ins[id] = &index{next: sm.raftLog.lastIndex() + 1}
 	sm.pendingConf = false
 }
 
@@ -482,9 +482,9 @@ func stepFollower(sm *stateMachine, m Message) bool {
 	case msgSnap:
 		sm.handleSnapshot(m)
 	case msgVote:
-		if (sm.vote == none || sm.vote == m.From) && sm.log.isUpToDate(m.Index, m.LogTerm) {
+		if (sm.vote == none || sm.vote == m.From) && sm.raftLog.isUpToDate(m.Index, m.LogTerm) {
 			sm.vote = m.From
-			sm.send(Message{To: m.From, Type: msgVoteResp, Index: sm.log.lastIndex()})
+			sm.send(Message{To: m.From, Type: msgVoteResp, Index: sm.raftLog.lastIndex()})
 		} else {
 			sm.send(Message{To: m.From, Type: msgVoteResp, Index: -1})
 		}
@@ -495,11 +495,11 @@ func stepFollower(sm *stateMachine, m Message) bool {
 // maybeCompact tries to compact the log. It calls the snapshoter to take a snapshot and
 // then compact the log up-to the index at which the snapshot was taken.
 func (sm *stateMachine) maybeCompact() bool {
-	if sm.snapshoter == nil || !sm.log.shouldCompact() {
+	if sm.snapshoter == nil || !sm.raftLog.shouldCompact() {
 		return false
 	}
-	sm.snapshoter.Snap(sm.log.applied, sm.log.term(sm.log.applied), sm.nodes())
-	sm.log.compact(sm.log.applied)
+	sm.snapshoter.Snap(sm.raftLog.applied, sm.raftLog.term(sm.raftLog.applied), sm.nodes())
+	sm.raftLog.compact(sm.raftLog.applied)
 	return true
 }
 
@@ -511,13 +511,13 @@ func (sm *stateMachine) restore(s Snapshot) {
 		panic("try to restore from snapshot, but snapshoter is nil")
 	}
 
-	sm.log.restore(s.Index, s.Term)
-	sm.index.Set(sm.log.lastIndex())
+	sm.raftLog.restore(s.Index, s.Term)
+	sm.index.Set(sm.raftLog.lastIndex())
 	sm.ins = make(map[int64]*index)
 	for _, n := range s.Nodes {
-		sm.ins[n] = &index{next: sm.log.lastIndex() + 1}
+		sm.ins[n] = &index{next: sm.raftLog.lastIndex() + 1}
 		if n == sm.id {
-			sm.ins[n].match = sm.log.lastIndex()
+			sm.ins[n].match = sm.raftLog.lastIndex()
 		}
 	}
 	sm.pendingConf = false
@@ -525,7 +525,7 @@ func (sm *stateMachine) restore(s Snapshot) {
 }
 
 func (sm *stateMachine) needSnapshot(i int64) bool {
-	if i < sm.log.offset {
+	if i < sm.raftLog.offset {
 		if sm.snapshoter == nil {
 			panic("need snapshot but snapshoter is nil")
 		}

+ 66 - 66
raft/raft_test.go

@@ -74,8 +74,8 @@ func TestLogReplication(t *testing.T) {
 		for j, x := range tt.network.peers {
 			sm := x.(*stateMachine)
 
-			if sm.log.committed != tt.wcommitted {
-				t.Errorf("#%d.%d: committed = %d, want %d", i, j, sm.log.committed, tt.wcommitted)
+			if sm.raftLog.committed != tt.wcommitted {
+				t.Errorf("#%d.%d: committed = %d, want %d", i, j, sm.raftLog.committed, tt.wcommitted)
 			}
 
 			ents := make([]Entry, 0)
@@ -106,8 +106,8 @@ func TestSingleNodeCommit(t *testing.T) {
 	tt.send(Message{From: 0, To: 0, Type: msgProp, Entries: []Entry{{Data: []byte("some data")}}})
 
 	sm := tt.peers[0].(*stateMachine)
-	if sm.log.committed != 3 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 3)
+	if sm.raftLog.committed != 3 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 3)
 	}
 }
 
@@ -127,8 +127,8 @@ func TestCannotCommitWithoutNewTermEntry(t *testing.T) {
 	tt.send(Message{From: 0, To: 0, Type: msgProp, Entries: []Entry{{Data: []byte("some data")}}})
 
 	sm := tt.peers[0].(*stateMachine)
-	if sm.log.committed != 1 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 1)
+	if sm.raftLog.committed != 1 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 1)
 	}
 
 	// network recovery
@@ -141,8 +141,8 @@ func TestCannotCommitWithoutNewTermEntry(t *testing.T) {
 
 	// no log entries from previous term should be committed
 	sm = tt.peers[1].(*stateMachine)
-	if sm.log.committed != 1 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 1)
+	if sm.raftLog.committed != 1 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 1)
 	}
 
 	tt.recover()
@@ -152,15 +152,15 @@ func TestCannotCommitWithoutNewTermEntry(t *testing.T) {
 	// should be committed
 	tt.send(Message{From: 1, To: 1, Type: msgBeat})
 
-	if sm.log.committed != 4 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 4)
+	if sm.raftLog.committed != 4 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 4)
 	}
 
 	// still be able to append a entry
 	tt.send(Message{From: 1, To: 1, Type: msgProp, Entries: []Entry{{Data: []byte("some data")}}})
 
-	if sm.log.committed != 5 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 5)
+	if sm.raftLog.committed != 5 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 5)
 	}
 }
 
@@ -179,8 +179,8 @@ func TestCommitWithoutNewTermEntry(t *testing.T) {
 	tt.send(Message{From: 0, To: 0, Type: msgProp, Entries: []Entry{{Data: []byte("some data")}}})
 
 	sm := tt.peers[0].(*stateMachine)
-	if sm.log.committed != 1 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 1)
+	if sm.raftLog.committed != 1 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 1)
 	}
 
 	// network recovery
@@ -191,8 +191,8 @@ func TestCommitWithoutNewTermEntry(t *testing.T) {
 	// should be committed
 	tt.send(Message{From: 1, To: 1, Type: msgHup})
 
-	if sm.log.committed != 4 {
-		t.Errorf("committed = %d, want %d", sm.log.committed, 4)
+	if sm.raftLog.committed != 4 {
+		t.Errorf("committed = %d, want %d", sm.raftLog.committed, 4)
 	}
 }
 
@@ -210,12 +210,12 @@ func TestDuelingCandidates(t *testing.T) {
 	nt.recover()
 	nt.send(Message{From: 2, To: 2, Type: msgHup})
 
-	wlog := &log{ents: []Entry{{}, Entry{Type: Normal, Data: nil, Term: 1}}, committed: 1}
+	wlog := &raftLog{ents: []Entry{{}, Entry{Type: Normal, Data: nil, Term: 1}}, committed: 1}
 	tests := []struct {
-		sm    *stateMachine
-		state stateType
-		term  int64
-		log   *log
+		sm      *stateMachine
+		state   stateType
+		term    int64
+		raftLog *raftLog
 	}{
 		{a, stateFollower, 2, wlog},
 		{b, stateFollower, 2, wlog},
@@ -229,9 +229,9 @@ func TestDuelingCandidates(t *testing.T) {
 		if g := tt.sm.term.Get(); g != tt.term {
 			t.Errorf("#%d: term = %d, want %d", i, g, tt.term)
 		}
-		base := ltoa(tt.log)
+		base := ltoa(tt.raftLog)
 		if sm, ok := nt.peers[int64(i)].(*stateMachine); ok {
-			l := ltoa(sm.log)
+			l := ltoa(sm.raftLog)
 			if g := diffu(base, l); g != "" {
 				t.Errorf("#%d: diff:\n%s", i, g)
 			}
@@ -262,10 +262,10 @@ func TestCandidateConcede(t *testing.T) {
 	if g := a.term; g != 1 {
 		t.Errorf("term = %d, want %d", g, 1)
 	}
-	wantLog := ltoa(&log{ents: []Entry{{}, {Type: Normal, Data: nil, Term: 1}, {Term: 1, Data: data}}, committed: 2})
+	wantLog := ltoa(&raftLog{ents: []Entry{{}, {Type: Normal, Data: nil, Term: 1}, {Term: 1, Data: data}}, committed: 2})
 	for i, p := range tt.peers {
 		if sm, ok := p.(*stateMachine); ok {
-			l := ltoa(sm.log)
+			l := ltoa(sm.raftLog)
 			if g := diffu(wantLog, l); g != "" {
 				t.Errorf("#%d: diff:\n%s", i, g)
 			}
@@ -294,7 +294,7 @@ func TestOldMessages(t *testing.T) {
 	// pretend we're an old leader trying to make progress
 	tt.send(Message{From: 0, To: 0, Type: msgApp, Term: 1, Entries: []Entry{{Term: 1}}})
 
-	l := &log{
+	l := &raftLog{
 		ents: []Entry{
 			{}, {Type: Normal, Data: nil, Term: 1},
 			{Type: Normal, Data: nil, Term: 2}, {Type: Normal, Data: nil, Term: 3},
@@ -304,7 +304,7 @@ func TestOldMessages(t *testing.T) {
 	base := ltoa(l)
 	for i, p := range tt.peers {
 		if sm, ok := p.(*stateMachine); ok {
-			l := ltoa(sm.log)
+			l := ltoa(sm.raftLog)
 			if g := diffu(base, l); g != "" {
 				t.Errorf("#%d: diff:\n%s", i, g)
 			}
@@ -351,12 +351,12 @@ func TestProposal(t *testing.T) {
 
 		wantLog := newLog()
 		if tt.success {
-			wantLog = &log{ents: []Entry{{}, {Type: Normal, Data: nil, Term: 1}, {Term: 1, Data: data}}, committed: 2}
+			wantLog = &raftLog{ents: []Entry{{}, {Type: Normal, Data: nil, Term: 1}, {Term: 1, Data: data}}, committed: 2}
 		}
 		base := ltoa(wantLog)
 		for i, p := range tt.peers {
 			if sm, ok := p.(*stateMachine); ok {
-				l := ltoa(sm.log)
+				l := ltoa(sm.raftLog)
 				if g := diffu(base, l); g != "" {
 					t.Errorf("#%d: diff:\n%s", i, g)
 				}
@@ -385,11 +385,11 @@ func TestProposalByProxy(t *testing.T) {
 		// propose via follower
 		tt.send(Message{From: 1, To: 1, Type: msgProp, Entries: []Entry{{Data: []byte("somedata")}}})
 
-		wantLog := &log{ents: []Entry{{}, {Type: Normal, Data: nil, Term: 1}, {Term: 1, Data: data}}, committed: 2}
+		wantLog := &raftLog{ents: []Entry{{}, {Type: Normal, Data: nil, Term: 1}, {Term: 1, Data: data}}, committed: 2}
 		base := ltoa(wantLog)
 		for i, p := range tt.peers {
 			if sm, ok := p.(*stateMachine); ok {
-				l := ltoa(sm.log)
+				l := ltoa(sm.raftLog)
 				if g := diffu(base, l); g != "" {
 					t.Errorf("#%d: diff:\n%s", i, g)
 				}
@@ -437,9 +437,9 @@ func TestCommit(t *testing.T) {
 		for j := 0; j < len(tt.matches); j++ {
 			ins[int64(j)] = &index{tt.matches[j], tt.matches[j] + 1}
 		}
-		sm := &stateMachine{log: &log{ents: tt.logs}, ins: ins, term: atomicInt(tt.smTerm)}
+		sm := &stateMachine{raftLog: &raftLog{ents: tt.logs}, ins: ins, term: atomicInt(tt.smTerm)}
 		sm.maybeCommit()
-		if g := sm.log.committed; g != tt.w {
+		if g := sm.raftLog.committed; g != tt.w {
 			t.Errorf("#%d: committed = %d, want %d", i, g, tt.w)
 		}
 	}
@@ -475,17 +475,17 @@ func TestHandleMsgApp(t *testing.T) {
 
 	for i, tt := range tests {
 		sm := &stateMachine{
-			state: stateFollower,
-			term:  2,
-			log:   &log{committed: 0, ents: []Entry{{}, {Term: 1}, {Term: 2}}},
+			state:   stateFollower,
+			term:    2,
+			raftLog: &raftLog{committed: 0, ents: []Entry{{}, {Term: 1}, {Term: 2}}},
 		}
 
 		sm.handleAppendEntries(tt.m)
-		if sm.log.lastIndex() != tt.wIndex {
-			t.Errorf("#%d: lastIndex = %d, want %d", i, sm.log.lastIndex(), tt.wIndex)
+		if sm.raftLog.lastIndex() != tt.wIndex {
+			t.Errorf("#%d: lastIndex = %d, want %d", i, sm.raftLog.lastIndex(), tt.wIndex)
 		}
-		if sm.log.committed != tt.wCommit {
-			t.Errorf("#%d: committed = %d, want %d", i, sm.log.committed, tt.wCommit)
+		if sm.raftLog.committed != tt.wCommit {
+			t.Errorf("#%d: committed = %d, want %d", i, sm.raftLog.committed, tt.wCommit)
 		}
 		m := sm.Msgs()
 		if len(m) != 1 {
@@ -537,9 +537,9 @@ func TestRecvMsgVote(t *testing.T) {
 
 	for i, tt := range tests {
 		sm := &stateMachine{
-			state: tt.state,
-			vote:  tt.voteFor,
-			log:   &log{ents: []Entry{{}, {Term: 2}, {Term: 2}}},
+			state:   tt.state,
+			vote:    tt.voteFor,
+			raftLog: &raftLog{ents: []Entry{{}, {Term: 2}, {Term: 2}}},
 		}
 
 		sm.Step(Message{Type: msgVote, From: 1, Index: tt.i, LogTerm: tt.term})
@@ -614,20 +614,20 @@ func TestConf(t *testing.T) {
 	sm.becomeLeader()
 
 	sm.Step(Message{From: 0, To: 0, Type: msgProp, Entries: []Entry{{Type: AddNode}}})
-	if sm.log.lastIndex() != 2 {
-		t.Errorf("lastindex = %d, want %d", sm.log.lastIndex(), 1)
+	if sm.raftLog.lastIndex() != 2 {
+		t.Errorf("lastindex = %d, want %d", sm.raftLog.lastIndex(), 1)
 	}
 	if !sm.pendingConf {
 		t.Errorf("pendingConf = %v, want %v", sm.pendingConf, true)
 	}
-	if sm.log.ents[2].Type != AddNode {
-		t.Errorf("type = %d, want %d", sm.log.ents[1].Type, AddNode)
+	if sm.raftLog.ents[2].Type != AddNode {
+		t.Errorf("type = %d, want %d", sm.raftLog.ents[1].Type, AddNode)
 	}
 
 	// deny the second configuration change request if there is a pending one
 	sm.Step(Message{From: 0, To: 0, Type: msgProp, Entries: []Entry{{Type: AddNode}}})
-	if sm.log.lastIndex() != 2 {
-		t.Errorf("lastindex = %d, want %d", sm.log.lastIndex(), 1)
+	if sm.raftLog.lastIndex() != 2 {
+		t.Errorf("lastindex = %d, want %d", sm.raftLog.lastIndex(), 1)
 	}
 }
 
@@ -645,7 +645,7 @@ func TestConfChangeLeader(t *testing.T) {
 
 	for i, tt := range tests {
 		sm := newStateMachine(0, []int64{0})
-		sm.log = &log{ents: []Entry{{}, {Type: tt.et}}}
+		sm.raftLog = &raftLog{ents: []Entry{{}, {Type: tt.et}}}
 
 		sm.becomeCandidate()
 		sm.becomeLeader()
@@ -693,8 +693,8 @@ func TestAllServerStepdown(t *testing.T) {
 			if sm.term.Get() != tt.wterm {
 				t.Errorf("#%d.%d term = %v , want %v", i, j, sm.term.Get(), tt.wterm)
 			}
-			if int64(len(sm.log.ents)) != tt.windex {
-				t.Errorf("#%d.%d index = %v , want %v", i, j, len(sm.log.ents), tt.windex)
+			if int64(len(sm.raftLog.ents)) != tt.windex {
+				t.Errorf("#%d.%d index = %v , want %v", i, j, len(sm.raftLog.ents), tt.windex)
 			}
 			wlead := int64(1)
 			if msgType == msgVote {
@@ -722,7 +722,7 @@ func TestLeaderAppResp(t *testing.T) {
 		// sm term is 1 after it becomes the leader.
 		// thus the last log term must be 1 to be committed.
 		sm := newStateMachine(0, []int64{0, 1, 2})
-		sm.log = &log{ents: []Entry{{}, {Term: 0}, {Term: 1}}}
+		sm.raftLog = &raftLog{ents: []Entry{{}, {Term: 0}, {Term: 1}}}
 		sm.becomeCandidate()
 		sm.becomeLeader()
 		sm.Msgs()
@@ -757,7 +757,7 @@ func TestRecvMsgBeat(t *testing.T) {
 
 	for i, tt := range tests {
 		sm := newStateMachine(0, []int64{0, 1, 2})
-		sm.log = &log{ents: []Entry{{}, {Term: 0}, {Term: 1}}}
+		sm.raftLog = &raftLog{ents: []Entry{{}, {Term: 0}, {Term: 1}}}
 		sm.term.Set(1)
 		sm.state = tt.state
 		sm.Step(Message{From: 0, To: 0, Type: msgBeat})
@@ -789,10 +789,10 @@ func TestMaybeCompact(t *testing.T) {
 		sm := newStateMachine(0, []int64{0, 1, 2})
 		sm.setSnapshoter(tt.snapshoter)
 		for i := 0; i < defaultCompactThreshold*2; i++ {
-			sm.log.append(int64(i), Entry{Term: int64(i + 1)})
+			sm.raftLog.append(int64(i), Entry{Term: int64(i + 1)})
 		}
-		sm.log.applied = tt.applied
-		sm.log.committed = tt.applied
+		sm.raftLog.applied = tt.applied
+		sm.raftLog.committed = tt.applied
 
 		if g := sm.maybeCompact(); g != tt.wCompact {
 			t.Errorf("#%d: compact = %v, want %v", i, g, tt.wCompact)
@@ -848,11 +848,11 @@ func TestRestore(t *testing.T) {
 			sm.setSnapshoter(tt.snapshoter)
 			sm.restore(s)
 
-			if sm.log.lastIndex() != s.Index {
-				t.Errorf("#%d: log.lastIndex = %d, want %d", i, sm.log.lastIndex(), s.Index)
+			if sm.raftLog.lastIndex() != s.Index {
+				t.Errorf("#%d: log.lastIndex = %d, want %d", i, sm.raftLog.lastIndex(), s.Index)
 			}
-			if sm.log.term(s.Index) != s.Term {
-				t.Errorf("#%d: log.lastTerm = %d, want %d", i, sm.log.term(s.Index), s.Term)
+			if sm.raftLog.term(s.Index) != s.Term {
+				t.Errorf("#%d: log.lastTerm = %d, want %d", i, sm.raftLog.term(s.Index), s.Term)
 			}
 			sg := int64Slice(sm.nodes())
 			sw := int64Slice(s.Nodes)
@@ -895,7 +895,7 @@ func TestProvideSnap(t *testing.T) {
 
 	// force set the next of node 1, so that
 	// node 1 needs a snapshot
-	sm.ins[1].next = sm.log.offset
+	sm.ins[1].next = sm.raftLog.offset
 
 	sm.Step(Message{From: 1, To: 0, Type: msgAppResp, Index: -1})
 	msgs = sm.Msgs()
@@ -947,10 +947,10 @@ func TestSlowNodeRestore(t *testing.T) {
 		t.Errorf("follower.snap = %+v, want %+v", follower.snapshoter.GetSnap(), lead.snapshoter.GetSnap())
 	}
 
-	committed := follower.log.lastIndex()
+	committed := follower.raftLog.lastIndex()
 	nt.send(Message{From: 0, To: 0, Type: msgProp, Entries: []Entry{{}}})
-	if follower.log.committed != committed+1 {
-		t.Errorf("follower.comitted = %d, want %d", follower.log.committed, committed+1)
+	if follower.raftLog.committed != committed+1 {
+		t.Errorf("follower.comitted = %d, want %d", follower.raftLog.committed, committed+1)
 	}
 }
 
@@ -960,7 +960,7 @@ func ents(terms ...int64) *stateMachine {
 		ents = append(ents, Entry{Term: term})
 	}
 
-	sm := &stateMachine{log: &log{ents: ents}}
+	sm := &stateMachine{raftLog: &raftLog{ents: ents}}
 	sm.reset(0)
 	return sm
 }