log.go 17 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619
  1. package raft
  2. import (
  3. "bufio"
  4. "errors"
  5. "fmt"
  6. "io"
  7. "os"
  8. "sync"
  9. "github.com/coreos/etcd/third_party/github.com/goraft/raft/protobuf"
  10. )
  11. //------------------------------------------------------------------------------
  12. //
  13. // Typedefs
  14. //
  15. //------------------------------------------------------------------------------
  16. // A log is a collection of log entries that are persisted to durable storage.
  17. type Log struct {
  18. ApplyFunc func(*LogEntry, Command) (interface{}, error)
  19. file *os.File
  20. path string
  21. entries []*LogEntry
  22. commitIndex uint64
  23. mutex sync.RWMutex
  24. startIndex uint64 // the index before the first entry in the Log entries
  25. startTerm uint64
  26. }
  27. // The results of the applying a log entry.
  28. type logResult struct {
  29. returnValue interface{}
  30. err error
  31. }
  32. //------------------------------------------------------------------------------
  33. //
  34. // Constructor
  35. //
  36. //------------------------------------------------------------------------------
  37. // Creates a new log.
  38. func newLog() *Log {
  39. return &Log{
  40. entries: make([]*LogEntry, 0),
  41. }
  42. }
  43. //------------------------------------------------------------------------------
  44. //
  45. // Accessors
  46. //
  47. //------------------------------------------------------------------------------
  48. //--------------------------------------
  49. // Log Indices
  50. //--------------------------------------
  51. // The last committed index in the log.
  52. func (l *Log) CommitIndex() uint64 {
  53. l.mutex.RLock()
  54. defer l.mutex.RUnlock()
  55. return l.commitIndex
  56. }
  57. // The current index in the log.
  58. func (l *Log) currentIndex() uint64 {
  59. l.mutex.RLock()
  60. defer l.mutex.RUnlock()
  61. return l.internalCurrentIndex()
  62. }
  63. // The current index in the log without locking
  64. func (l *Log) internalCurrentIndex() uint64 {
  65. if len(l.entries) == 0 {
  66. return l.startIndex
  67. }
  68. return l.entries[len(l.entries)-1].Index()
  69. }
  70. // The next index in the log.
  71. func (l *Log) nextIndex() uint64 {
  72. return l.currentIndex() + 1
  73. }
  74. // Determines if the log contains zero entries.
  75. func (l *Log) isEmpty() bool {
  76. l.mutex.RLock()
  77. defer l.mutex.RUnlock()
  78. return (len(l.entries) == 0) && (l.startIndex == 0)
  79. }
  80. // The name of the last command in the log.
  81. func (l *Log) lastCommandName() string {
  82. l.mutex.RLock()
  83. defer l.mutex.RUnlock()
  84. if len(l.entries) > 0 {
  85. if entry := l.entries[len(l.entries)-1]; entry != nil {
  86. return entry.CommandName()
  87. }
  88. }
  89. return ""
  90. }
  91. //--------------------------------------
  92. // Log Terms
  93. //--------------------------------------
  94. // The current term in the log.
  95. func (l *Log) currentTerm() uint64 {
  96. l.mutex.RLock()
  97. defer l.mutex.RUnlock()
  98. if len(l.entries) == 0 {
  99. return l.startTerm
  100. }
  101. return l.entries[len(l.entries)-1].Term()
  102. }
  103. //------------------------------------------------------------------------------
  104. //
  105. // Methods
  106. //
  107. //------------------------------------------------------------------------------
  108. //--------------------------------------
  109. // State
  110. //--------------------------------------
  111. // Opens the log file and reads existing entries. The log can remain open and
  112. // continue to append entries to the end of the log.
  113. func (l *Log) open(path string) error {
  114. // Read all the entries from the log if one exists.
  115. var readBytes int64
  116. var err error
  117. debugln("log.open.open ", path)
  118. // open log file
  119. l.file, err = os.OpenFile(path, os.O_RDWR, 0600)
  120. l.path = path
  121. if err != nil {
  122. // if the log file does not exist before
  123. // we create the log file and set commitIndex to 0
  124. if os.IsNotExist(err) {
  125. l.file, err = os.OpenFile(path, os.O_WRONLY|os.O_CREATE, 0600)
  126. debugln("log.open.create ", path)
  127. return err
  128. }
  129. return err
  130. }
  131. debugln("log.open.exist ", path)
  132. // Read the file and decode entries.
  133. for {
  134. // Instantiate log entry and decode into it.
  135. entry, _ := newLogEntry(l, nil, 0, 0, nil)
  136. entry.Position, _ = l.file.Seek(0, os.SEEK_CUR)
  137. n, err := entry.Decode(l.file)
  138. if err != nil {
  139. if err == io.EOF {
  140. debugln("open.log.append: finish ")
  141. } else {
  142. if err = os.Truncate(path, readBytes); err != nil {
  143. return fmt.Errorf("raft.Log: Unable to recover: %v", err)
  144. }
  145. }
  146. break
  147. }
  148. if entry.Index() > l.startIndex {
  149. // Append entry.
  150. l.entries = append(l.entries, entry)
  151. if entry.Index() <= l.commitIndex {
  152. command, err := newCommand(entry.CommandName(), entry.Command())
  153. if err != nil {
  154. continue
  155. }
  156. l.ApplyFunc(entry, command)
  157. }
  158. debugln("open.log.append log index ", entry.Index())
  159. }
  160. readBytes += int64(n)
  161. }
  162. debugln("open.log.recovery number of log ", len(l.entries))
  163. return nil
  164. }
  165. // Closes the log file.
  166. func (l *Log) close() {
  167. l.mutex.Lock()
  168. defer l.mutex.Unlock()
  169. if l.file != nil {
  170. l.file.Close()
  171. l.file = nil
  172. }
  173. l.entries = make([]*LogEntry, 0)
  174. }
  175. // sync to disk
  176. func (l *Log) sync() error {
  177. return l.file.Sync()
  178. }
  179. //--------------------------------------
  180. // Entries
  181. //--------------------------------------
  182. // Creates a log entry associated with this log.
  183. func (l *Log) createEntry(term uint64, command Command, e *ev) (*LogEntry, error) {
  184. return newLogEntry(l, e, l.nextIndex(), term, command)
  185. }
  186. // Retrieves an entry from the log. If the entry has been eliminated because
  187. // of a snapshot then nil is returned.
  188. func (l *Log) getEntry(index uint64) *LogEntry {
  189. l.mutex.RLock()
  190. defer l.mutex.RUnlock()
  191. if index <= l.startIndex || index > (l.startIndex+uint64(len(l.entries))) {
  192. return nil
  193. }
  194. return l.entries[index-l.startIndex-1]
  195. }
  196. // Checks if the log contains a given index/term combination.
  197. func (l *Log) containsEntry(index uint64, term uint64) bool {
  198. entry := l.getEntry(index)
  199. return (entry != nil && entry.Term() == term)
  200. }
  201. // Retrieves a list of entries after a given index as well as the term of the
  202. // index provided. A nil list of entries is returned if the index no longer
  203. // exists because a snapshot was made.
  204. func (l *Log) getEntriesAfter(index uint64, maxLogEntriesPerRequest uint64) ([]*LogEntry, uint64) {
  205. l.mutex.RLock()
  206. defer l.mutex.RUnlock()
  207. // Return nil if index is before the start of the log.
  208. if index < l.startIndex {
  209. traceln("log.entriesAfter.before: ", index, " ", l.startIndex)
  210. return nil, 0
  211. }
  212. // Return an error if the index doesn't exist.
  213. if index > (uint64(len(l.entries)) + l.startIndex) {
  214. panic(fmt.Sprintf("raft: Index is beyond end of log: %v %v", len(l.entries), index))
  215. }
  216. // If we're going from the beginning of the log then return the whole log.
  217. if index == l.startIndex {
  218. traceln("log.entriesAfter.beginning: ", index, " ", l.startIndex)
  219. return l.entries, l.startTerm
  220. }
  221. traceln("log.entriesAfter.partial: ", index, " ", l.entries[len(l.entries)-1].Index)
  222. entries := l.entries[index-l.startIndex:]
  223. length := len(entries)
  224. traceln("log.entriesAfter: startIndex:", l.startIndex, " length", len(l.entries))
  225. if uint64(length) < maxLogEntriesPerRequest {
  226. // Determine the term at the given entry and return a subslice.
  227. return entries, l.entries[index-1-l.startIndex].Term()
  228. } else {
  229. return entries[:maxLogEntriesPerRequest], l.entries[index-1-l.startIndex].Term()
  230. }
  231. }
  232. //--------------------------------------
  233. // Commit
  234. //--------------------------------------
  235. // Retrieves the last index and term that has been committed to the log.
  236. func (l *Log) commitInfo() (index uint64, term uint64) {
  237. l.mutex.RLock()
  238. defer l.mutex.RUnlock()
  239. // If we don't have any committed entries then just return zeros.
  240. if l.commitIndex == 0 {
  241. return 0, 0
  242. }
  243. // No new commit log after snapshot
  244. if l.commitIndex == l.startIndex {
  245. return l.startIndex, l.startTerm
  246. }
  247. // Return the last index & term from the last committed entry.
  248. debugln("commitInfo.get.[", l.commitIndex, "/", l.startIndex, "]")
  249. entry := l.entries[l.commitIndex-1-l.startIndex]
  250. return entry.Index(), entry.Term()
  251. }
  252. // Retrieves the last index and term that has been appended to the log.
  253. func (l *Log) lastInfo() (index uint64, term uint64) {
  254. l.mutex.RLock()
  255. defer l.mutex.RUnlock()
  256. // If we don't have any entries then just return zeros.
  257. if len(l.entries) == 0 {
  258. return l.startIndex, l.startTerm
  259. }
  260. // Return the last index & term
  261. entry := l.entries[len(l.entries)-1]
  262. return entry.Index(), entry.Term()
  263. }
  264. // Updates the commit index
  265. func (l *Log) updateCommitIndex(index uint64) {
  266. l.mutex.Lock()
  267. defer l.mutex.Unlock()
  268. if index > l.commitIndex {
  269. l.commitIndex = index
  270. }
  271. debugln("update.commit.index ", index)
  272. }
  273. // Updates the commit index and writes entries after that index to the stable storage.
  274. func (l *Log) setCommitIndex(index uint64) error {
  275. l.mutex.Lock()
  276. defer l.mutex.Unlock()
  277. // this is not error any more after limited the number of sending entries
  278. // commit up to what we already have
  279. if index > l.startIndex+uint64(len(l.entries)) {
  280. debugln("raft.Log: Commit index", index, "set back to ", len(l.entries))
  281. index = l.startIndex + uint64(len(l.entries))
  282. }
  283. // Do not allow previous indices to be committed again.
  284. // This could happens, since the guarantee is that the new leader has up-to-dated
  285. // log entries rather than has most up-to-dated committed index
  286. // For example, Leader 1 send log 80 to follower 2 and follower 3
  287. // follower 2 and follow 3 all got the new entries and reply
  288. // leader 1 committed entry 80 and send reply to follower 2 and follower3
  289. // follower 2 receive the new committed index and update committed index to 80
  290. // leader 1 fail to send the committed index to follower 3
  291. // follower 3 promote to leader (server 1 and server 2 will vote, since leader 3
  292. // has up-to-dated the entries)
  293. // when new leader 3 send heartbeat with committed index = 0 to follower 2,
  294. // follower 2 should reply success and let leader 3 update the committed index to 80
  295. if index < l.commitIndex {
  296. return nil
  297. }
  298. // Find all entries whose index is between the previous index and the current index.
  299. for i := l.commitIndex + 1; i <= index; i++ {
  300. entryIndex := i - 1 - l.startIndex
  301. entry := l.entries[entryIndex]
  302. // Update commit index.
  303. l.commitIndex = entry.Index()
  304. // Decode the command.
  305. command, err := newCommand(entry.CommandName(), entry.Command())
  306. if err != nil {
  307. return err
  308. }
  309. // Apply the changes to the state machine and store the error code.
  310. returnValue, err := l.ApplyFunc(entry, command)
  311. debugf("setCommitIndex.set.result index: %v, entries index: %v", i, entryIndex)
  312. if entry.event != nil {
  313. entry.event.returnValue = returnValue
  314. entry.event.c <- err
  315. }
  316. }
  317. return nil
  318. }
  319. // Set the commitIndex at the head of the log file to the current
  320. // commit Index. This should be called after obtained a log lock
  321. func (l *Log) flushCommitIndex() {
  322. l.file.Seek(0, os.SEEK_SET)
  323. fmt.Fprintf(l.file, "%8x\n", l.commitIndex)
  324. l.file.Seek(0, os.SEEK_END)
  325. }
  326. //--------------------------------------
  327. // Truncation
  328. //--------------------------------------
  329. // Truncates the log to the given index and term. This only works if the log
  330. // at the index has not been committed.
  331. func (l *Log) truncate(index uint64, term uint64) error {
  332. l.mutex.Lock()
  333. defer l.mutex.Unlock()
  334. debugln("log.truncate: ", index)
  335. // Do not allow committed entries to be truncated.
  336. if index < l.commitIndex {
  337. debugln("log.truncate.before")
  338. return fmt.Errorf("raft.Log: Index is already committed (%v): (IDX=%v, TERM=%v)", l.commitIndex, index, term)
  339. }
  340. // Do not truncate past end of entries.
  341. if index > l.startIndex+uint64(len(l.entries)) {
  342. debugln("log.truncate.after")
  343. return fmt.Errorf("raft.Log: Entry index does not exist (MAX=%v): (IDX=%v, TERM=%v)", len(l.entries), index, term)
  344. }
  345. // If we're truncating everything then just clear the entries.
  346. if index == l.startIndex {
  347. debugln("log.truncate.clear")
  348. l.file.Truncate(0)
  349. l.file.Seek(0, os.SEEK_SET)
  350. // notify clients if this node is the previous leader
  351. for _, entry := range l.entries {
  352. if entry.event != nil {
  353. entry.event.c <- errors.New("command failed to be committed due to node failure")
  354. }
  355. }
  356. l.entries = []*LogEntry{}
  357. } else {
  358. // Do not truncate if the entry at index does not have the matching term.
  359. entry := l.entries[index-l.startIndex-1]
  360. if len(l.entries) > 0 && entry.Term() != term {
  361. debugln("log.truncate.termMismatch")
  362. return fmt.Errorf("raft.Log: Entry at index does not have matching term (%v): (IDX=%v, TERM=%v)", entry.Term(), index, term)
  363. }
  364. // Otherwise truncate up to the desired entry.
  365. if index < l.startIndex+uint64(len(l.entries)) {
  366. debugln("log.truncate.finish")
  367. position := l.entries[index-l.startIndex].Position
  368. l.file.Truncate(position)
  369. l.file.Seek(position, os.SEEK_SET)
  370. // notify clients if this node is the previous leader
  371. for i := index - l.startIndex; i < uint64(len(l.entries)); i++ {
  372. entry := l.entries[i]
  373. if entry.event != nil {
  374. entry.event.c <- errors.New("command failed to be committed due to node failure")
  375. }
  376. }
  377. l.entries = l.entries[0 : index-l.startIndex]
  378. }
  379. }
  380. return nil
  381. }
  382. //--------------------------------------
  383. // Append
  384. //--------------------------------------
  385. // Appends a series of entries to the log.
  386. func (l *Log) appendEntries(entries []*protobuf.LogEntry) error {
  387. l.mutex.Lock()
  388. defer l.mutex.Unlock()
  389. startPosition, _ := l.file.Seek(0, os.SEEK_CUR)
  390. w := bufio.NewWriter(l.file)
  391. var size int64
  392. var err error
  393. // Append each entry but exit if we hit an error.
  394. for i := range entries {
  395. logEntry := &LogEntry{
  396. log: l,
  397. Position: startPosition,
  398. pb: entries[i],
  399. }
  400. if size, err = l.writeEntry(logEntry, w); err != nil {
  401. return err
  402. }
  403. startPosition += size
  404. }
  405. w.Flush()
  406. err = l.sync()
  407. if err != nil {
  408. panic(err)
  409. }
  410. return nil
  411. }
  412. // Writes a single log entry to the end of the log.
  413. func (l *Log) appendEntry(entry *LogEntry) error {
  414. l.mutex.Lock()
  415. defer l.mutex.Unlock()
  416. if l.file == nil {
  417. return errors.New("raft.Log: Log is not open")
  418. }
  419. // Make sure the term and index are greater than the previous.
  420. if len(l.entries) > 0 {
  421. lastEntry := l.entries[len(l.entries)-1]
  422. if entry.Term() < lastEntry.Term() {
  423. return fmt.Errorf("raft.Log: Cannot append entry with earlier term (%x:%x <= %x:%x)", entry.Term(), entry.Index(), lastEntry.Term(), lastEntry.Index())
  424. } else if entry.Term() == lastEntry.Term() && entry.Index() <= lastEntry.Index() {
  425. return fmt.Errorf("raft.Log: Cannot append entry with earlier index in the same term (%x:%x <= %x:%x)", entry.Term(), entry.Index(), lastEntry.Term(), lastEntry.Index())
  426. }
  427. }
  428. position, _ := l.file.Seek(0, os.SEEK_CUR)
  429. entry.Position = position
  430. // Write to storage.
  431. if _, err := entry.Encode(l.file); err != nil {
  432. return err
  433. }
  434. // Append to entries list if stored on disk.
  435. l.entries = append(l.entries, entry)
  436. return nil
  437. }
  438. // appendEntry with Buffered io
  439. func (l *Log) writeEntry(entry *LogEntry, w io.Writer) (int64, error) {
  440. if l.file == nil {
  441. return -1, errors.New("raft.Log: Log is not open")
  442. }
  443. // Make sure the term and index are greater than the previous.
  444. if len(l.entries) > 0 {
  445. lastEntry := l.entries[len(l.entries)-1]
  446. if entry.Term() < lastEntry.Term() {
  447. return -1, fmt.Errorf("raft.Log: Cannot append entry with earlier term (%x:%x <= %x:%x)", entry.Term(), entry.Index(), lastEntry.Term(), lastEntry.Index())
  448. } else if entry.Term() == lastEntry.Term() && entry.Index() <= lastEntry.Index() {
  449. return -1, fmt.Errorf("raft.Log: Cannot append entry with earlier index in the same term (%x:%x <= %x:%x)", entry.Term(), entry.Index(), lastEntry.Term(), lastEntry.Index())
  450. }
  451. }
  452. // Write to storage.
  453. size, err := entry.Encode(w)
  454. if err != nil {
  455. return -1, err
  456. }
  457. // Append to entries list if stored on disk.
  458. l.entries = append(l.entries, entry)
  459. return int64(size), nil
  460. }
  461. //--------------------------------------
  462. // Log compaction
  463. //--------------------------------------
  464. // compact the log before index (including index)
  465. func (l *Log) compact(index uint64, term uint64) error {
  466. var entries []*LogEntry
  467. l.mutex.Lock()
  468. defer l.mutex.Unlock()
  469. if index == 0 {
  470. return nil
  471. }
  472. // nothing to compaction
  473. // the index may be greater than the current index if
  474. // we just recovery from on snapshot
  475. if index >= l.internalCurrentIndex() {
  476. entries = make([]*LogEntry, 0)
  477. } else {
  478. // get all log entries after index
  479. entries = l.entries[index-l.startIndex:]
  480. }
  481. // create a new log file and add all the entries
  482. new_file_path := l.path + ".new"
  483. file, err := os.OpenFile(new_file_path, os.O_APPEND|os.O_CREATE|os.O_WRONLY, 0600)
  484. if err != nil {
  485. return err
  486. }
  487. for _, entry := range entries {
  488. position, _ := l.file.Seek(0, os.SEEK_CUR)
  489. entry.Position = position
  490. if _, err = entry.Encode(file); err != nil {
  491. file.Close()
  492. os.Remove(new_file_path)
  493. return err
  494. }
  495. }
  496. file.Sync()
  497. old_file := l.file
  498. // rename the new log file
  499. err = os.Rename(new_file_path, l.path)
  500. if err != nil {
  501. file.Close()
  502. os.Remove(new_file_path)
  503. return err
  504. }
  505. l.file = file
  506. // close the old log file
  507. old_file.Close()
  508. // compaction the in memory log
  509. l.entries = entries
  510. l.startIndex = index
  511. l.startTerm = term
  512. return nil
  513. }