server.go 8.4 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316
  1. package etcdserver
  2. import (
  3. "encoding/binary"
  4. "errors"
  5. "io"
  6. "log"
  7. "time"
  8. crand "crypto/rand"
  9. pb "github.com/coreos/etcd/etcdserver/etcdserverpb"
  10. "github.com/coreos/etcd/raft"
  11. "github.com/coreos/etcd/raft/raftpb"
  12. "github.com/coreos/etcd/store"
  13. "github.com/coreos/etcd/third_party/code.google.com/p/go.net/context"
  14. "github.com/coreos/etcd/wait"
  15. )
  16. const (
  17. defaultSyncTimeout = time.Second
  18. DefaultSnapCount = 10000
  19. )
  20. var (
  21. ErrUnknownMethod = errors.New("etcdserver: unknown method")
  22. ErrStopped = errors.New("etcdserver: server stopped")
  23. )
  24. type SendFunc func(m []raftpb.Message)
  25. type SaveFunc func(st raftpb.HardState, ents []raftpb.Entry)
  26. type Response struct {
  27. Event *store.Event
  28. Watcher store.Watcher
  29. err error
  30. }
  31. type Storage interface {
  32. // Save function saves ents and state to the underlying stable storage.
  33. // Save MUST block until st and ents are on stable storage.
  34. Save(st raftpb.HardState, ents []raftpb.Entry)
  35. // SaveSnap function saves snapshot to the underlying stable storage.
  36. SaveSnap(snap raftpb.Snapshot)
  37. // TODO: WAL should be able to control cut itself. After implement self-controled cut,
  38. // remove it in this interface.
  39. // Cut cuts out a new wal file for saving new state and entries.
  40. Cut() error
  41. }
  42. type Server interface {
  43. // Start performs any initialization of the Server necessary for it to
  44. // begin serving requests. It must be called before Do or Process.
  45. // Start must be non-blocking; any long-running server functionality
  46. // should be implemented in goroutines.
  47. Start()
  48. // Stop terminates the Server and performs any necessary finalization.
  49. // Do and Process cannot be called after Stop has been invoked.
  50. Stop()
  51. // Do takes a request and attempts to fulfil it, returning a Response.
  52. Do(ctx context.Context, r pb.Request) (Response, error)
  53. // Process takes a raft message and applies it to the server's raft state
  54. // machine, respecting any timeout of the given context.
  55. Process(ctx context.Context, m raftpb.Message) error
  56. }
  57. // EtcdServer is the production implementation of the Server interface
  58. type EtcdServer struct {
  59. w wait.Wait
  60. done chan struct{}
  61. Node raft.Node
  62. Store store.Store
  63. // Send specifies the send function for sending msgs to peers. Send
  64. // MUST NOT block. It is okay to drop messages, since clients should
  65. // timeout and reissue their messages. If Send is nil, server will
  66. // panic.
  67. Send SendFunc
  68. Storage Storage
  69. Ticker <-chan time.Time
  70. SyncTicker <-chan time.Time
  71. SnapCount int64 // number of entries to trigger a snapshot
  72. }
  73. // Start prepares and starts server in a new goroutine. It is no longer safe to
  74. // modify a server's fields after it has been sent to Start.
  75. func (s *EtcdServer) Start() {
  76. if s.SnapCount == 0 {
  77. log.Printf("etcdserver: set snapshot count to default %d", DefaultSnapCount)
  78. s.SnapCount = DefaultSnapCount
  79. }
  80. s.w = wait.New()
  81. s.done = make(chan struct{})
  82. go s.run()
  83. }
  84. func (s *EtcdServer) Process(ctx context.Context, m raftpb.Message) error {
  85. return s.Node.Step(ctx, m)
  86. }
  87. func (s *EtcdServer) run() {
  88. var syncC <-chan time.Time
  89. // snapi indicates the index of the last submitted snapshot request
  90. var snapi, appliedi int64
  91. for {
  92. select {
  93. case <-s.Ticker:
  94. s.Node.Tick()
  95. case rd := <-s.Node.Ready():
  96. s.Storage.Save(rd.HardState, rd.Entries)
  97. s.Storage.SaveSnap(rd.Snapshot)
  98. s.Send(rd.Messages)
  99. // TODO(bmizerany): do this in the background, but take
  100. // care to apply entries in a single goroutine, and not
  101. // race them.
  102. for _, e := range rd.CommittedEntries {
  103. var r pb.Request
  104. if err := r.Unmarshal(e.Data); err != nil {
  105. panic("TODO: this is bad, what do we do about it?")
  106. }
  107. s.w.Trigger(r.Id, s.apply(r))
  108. appliedi = e.Index
  109. }
  110. if rd.Snapshot.Index > snapi {
  111. snapi = rd.Snapshot.Index
  112. }
  113. // recover from snapshot if it is more updated than current applied
  114. if rd.Snapshot.Index > appliedi {
  115. if err := s.Store.Recovery(rd.Snapshot.Data); err != nil {
  116. panic("TODO: this is bad, what do we do about it?")
  117. }
  118. appliedi = rd.Snapshot.Index
  119. }
  120. if appliedi-snapi > s.SnapCount {
  121. s.snapshot()
  122. snapi = appliedi
  123. }
  124. if rd.SoftState != nil {
  125. if rd.RaftState == raft.StateLeader {
  126. syncC = s.SyncTicker
  127. } else {
  128. syncC = nil
  129. }
  130. }
  131. case <-syncC:
  132. s.sync(defaultSyncTimeout)
  133. case <-s.done:
  134. return
  135. }
  136. }
  137. }
  138. // Stop stops the server, and shuts down the running goroutine. Stop should be
  139. // called after a Start(s), otherwise it will block forever.
  140. func (s *EtcdServer) Stop() {
  141. s.Node.Stop()
  142. close(s.done)
  143. }
  144. // Do interprets r and performs an operation on s.Store according to r.Method
  145. // and other fields. If r.Method is "POST", "PUT", "DELETE", or a "GET" with
  146. // Quorum == true, r will be sent through consensus before performing its
  147. // respective operation. Do will block until an action is performed or there is
  148. // an error.
  149. func (s *EtcdServer) Do(ctx context.Context, r pb.Request) (Response, error) {
  150. if r.Id == 0 {
  151. panic("r.Id cannot be 0")
  152. }
  153. if r.Method == "GET" && r.Quorum {
  154. r.Method = "QGET"
  155. }
  156. switch r.Method {
  157. case "POST", "PUT", "DELETE", "QGET":
  158. data, err := r.Marshal()
  159. if err != nil {
  160. return Response{}, err
  161. }
  162. ch := s.w.Register(r.Id)
  163. s.Node.Propose(ctx, data)
  164. select {
  165. case x := <-ch:
  166. resp := x.(Response)
  167. return resp, resp.err
  168. case <-ctx.Done():
  169. s.w.Trigger(r.Id, nil) // GC wait
  170. return Response{}, ctx.Err()
  171. case <-s.done:
  172. return Response{}, ErrStopped
  173. }
  174. case "GET":
  175. switch {
  176. case r.Wait:
  177. wc, err := s.Store.Watch(r.Path, r.Recursive, false, r.Since)
  178. if err != nil {
  179. return Response{}, err
  180. }
  181. return Response{Watcher: wc}, nil
  182. default:
  183. ev, err := s.Store.Get(r.Path, r.Recursive, r.Sorted)
  184. if err != nil {
  185. return Response{}, err
  186. }
  187. return Response{Event: ev}, nil
  188. }
  189. default:
  190. return Response{}, ErrUnknownMethod
  191. }
  192. }
  193. // sync proposes a SYNC request and is non-blocking.
  194. // This makes no guarantee that the request will be proposed or performed.
  195. // The request will be cancelled after the given timeout.
  196. func (s *EtcdServer) sync(timeout time.Duration) {
  197. ctx, cancel := context.WithTimeout(context.Background(), timeout)
  198. req := pb.Request{
  199. Method: "SYNC",
  200. Id: GenID(),
  201. Time: time.Now().UnixNano(),
  202. }
  203. data, err := req.Marshal()
  204. if err != nil {
  205. log.Printf("marshal request %#v error: %v", req, err)
  206. return
  207. }
  208. // There is no promise that node has leader when do SYNC request,
  209. // so it uses goroutine to propose.
  210. go func() {
  211. s.Node.Propose(ctx, data)
  212. cancel()
  213. }()
  214. }
  215. // apply interprets r as a call to store.X and returns an Response interpreted from store.Event
  216. func (s *EtcdServer) apply(r pb.Request) Response {
  217. f := func(ev *store.Event, err error) Response {
  218. return Response{Event: ev, err: err}
  219. }
  220. expr := time.Unix(0, r.Expiration)
  221. switch r.Method {
  222. case "POST":
  223. return f(s.Store.Create(r.Path, r.Dir, r.Val, true, expr))
  224. case "PUT":
  225. exists, existsSet := getBool(r.PrevExists)
  226. switch {
  227. case existsSet:
  228. if exists {
  229. return f(s.Store.Update(r.Path, r.Val, expr))
  230. } else {
  231. return f(s.Store.Create(r.Path, r.Dir, r.Val, false, expr))
  232. }
  233. case r.PrevIndex > 0 || r.PrevValue != "":
  234. return f(s.Store.CompareAndSwap(r.Path, r.PrevValue, r.PrevIndex, r.Val, expr))
  235. default:
  236. return f(s.Store.Set(r.Path, r.Dir, r.Val, expr))
  237. }
  238. case "DELETE":
  239. switch {
  240. case r.PrevIndex > 0 || r.PrevValue != "":
  241. return f(s.Store.CompareAndDelete(r.Path, r.PrevValue, r.PrevIndex))
  242. default:
  243. return f(s.Store.Delete(r.Path, r.Recursive, r.Dir))
  244. }
  245. case "QGET":
  246. return f(s.Store.Get(r.Path, r.Recursive, r.Sorted))
  247. case "SYNC":
  248. s.Store.DeleteExpiredKeys(time.Unix(0, r.Time))
  249. return Response{}
  250. default:
  251. // This should never be reached, but just in case:
  252. return Response{err: ErrUnknownMethod}
  253. }
  254. }
  255. // TODO: non-blocking snapshot
  256. func (s *EtcdServer) snapshot() {
  257. d, err := s.Store.Save()
  258. // TODO: current store will never fail to do a snapshot
  259. // what should we do if the store might fail?
  260. if err != nil {
  261. panic("TODO: this is bad, what do we do about it?")
  262. }
  263. s.Node.Compact(d)
  264. s.Storage.Cut()
  265. }
  266. // TODO: move the function to /id pkg maybe?
  267. // GenID generates a random id that is not equal to 0.
  268. func GenID() int64 {
  269. for {
  270. b := make([]byte, 8)
  271. if _, err := io.ReadFull(crand.Reader, b); err != nil {
  272. panic(err) // really bad stuff happened
  273. }
  274. n := int64(binary.BigEndian.Uint64(b))
  275. if n != 0 {
  276. return n
  277. }
  278. }
  279. }
  280. func getBool(v *bool) (vv bool, set bool) {
  281. if v == nil {
  282. return false, false
  283. }
  284. return *v, true
  285. }