etcd.go 13 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425
  1. /*
  2. Copyright 2013 CoreOS Inc.
  3. Licensed under the Apache License, Version 2.0 (the "License");
  4. you may not use this file except in compliance with the License.
  5. You may obtain a copy of the License at
  6. http://www.apache.org/licenses/LICENSE-2.0
  7. Unless required by applicable law or agreed to in writing, software
  8. distributed under the License is distributed on an "AS IS" BASIS,
  9. WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  10. See the License for the specific language governing permissions and
  11. limitations under the License.
  12. */
  13. package etcd
  14. import (
  15. "net/http"
  16. "os"
  17. "path/filepath"
  18. "runtime"
  19. "strings"
  20. "sync"
  21. "time"
  22. goetcd "github.com/coreos/etcd/third_party/github.com/coreos/go-etcd/etcd"
  23. golog "github.com/coreos/etcd/third_party/github.com/coreos/go-log/log"
  24. "github.com/coreos/etcd/third_party/github.com/goraft/raft"
  25. httpclient "github.com/coreos/etcd/third_party/github.com/mreiferson/go-httpclient"
  26. "github.com/coreos/etcd/config"
  27. ehttp "github.com/coreos/etcd/http"
  28. "github.com/coreos/etcd/log"
  29. "github.com/coreos/etcd/metrics"
  30. "github.com/coreos/etcd/server"
  31. "github.com/coreos/etcd/store"
  32. )
  33. // TODO(yichengq): constant extraTimeout is a hack.
  34. // Current problem is that there is big lag between join command
  35. // execution and join success.
  36. // Fix it later. It should be removed when proper method is found and
  37. // enough tests are provided. It is expected to be calculated from
  38. // heartbeatInterval and electionTimeout only.
  39. const extraTimeout = time.Duration(1000) * time.Millisecond
  40. type Etcd struct {
  41. Config *config.Config // etcd config
  42. Store store.Store // data store
  43. Registry *server.Registry // stores URL information for nodes
  44. Server *server.Server // http server, runs on 4001 by default
  45. PeerServer *server.PeerServer // peer server, runs on 7001 by default
  46. StandbyServer *server.StandbyServer
  47. server *http.Server
  48. peerServer *http.Server
  49. mode Mode
  50. modeMutex sync.Mutex
  51. closeChan chan bool
  52. readyNotify chan bool // To signal when server is ready to accept connections
  53. onceReady sync.Once
  54. stopNotify chan bool // To signal when server is stopped totally
  55. }
  56. // New returns a new Etcd instance.
  57. func New(c *config.Config) *Etcd {
  58. if c == nil {
  59. c = config.New()
  60. }
  61. return &Etcd{
  62. Config: c,
  63. closeChan: make(chan bool),
  64. readyNotify: make(chan bool),
  65. stopNotify: make(chan bool),
  66. }
  67. }
  68. // Run the etcd instance.
  69. func (e *Etcd) Run() {
  70. // Sanitize all the input fields.
  71. if err := e.Config.Sanitize(); err != nil {
  72. log.Fatalf("failed sanitizing configuration: %v", err)
  73. }
  74. // Force remove server configuration if specified.
  75. if e.Config.Force {
  76. e.Config.Reset()
  77. }
  78. // Enable options.
  79. if e.Config.VeryVeryVerbose {
  80. log.Verbose = true
  81. raft.SetLogLevel(raft.Trace)
  82. goetcd.SetLogger(
  83. golog.New(
  84. "go-etcd",
  85. false,
  86. golog.CombinedSink(
  87. os.Stdout,
  88. "[%s] %s %-9s | %s\n",
  89. []string{"prefix", "time", "priority", "message"},
  90. ),
  91. ),
  92. )
  93. } else if e.Config.VeryVerbose {
  94. log.Verbose = true
  95. raft.SetLogLevel(raft.Debug)
  96. } else if e.Config.Verbose {
  97. log.Verbose = true
  98. }
  99. if e.Config.CPUProfileFile != "" {
  100. profile(e.Config.CPUProfileFile)
  101. }
  102. if e.Config.DataDir == "" {
  103. log.Fatal("The data dir was not set and could not be guessed from machine name")
  104. }
  105. // Create data directory if it doesn't already exist.
  106. if err := os.MkdirAll(e.Config.DataDir, 0744); err != nil {
  107. log.Fatalf("Unable to create path: %s", err)
  108. }
  109. // Warn people if they have an info file
  110. info := filepath.Join(e.Config.DataDir, "info")
  111. if _, err := os.Stat(info); err == nil {
  112. log.Warnf("All cached configuration is now ignored. The file %s can be removed.", info)
  113. }
  114. var mbName string
  115. if e.Config.Trace() {
  116. mbName = e.Config.MetricsBucketName()
  117. runtime.SetBlockProfileRate(1)
  118. }
  119. mb := metrics.NewBucket(mbName)
  120. if e.Config.GraphiteHost != "" {
  121. err := mb.Publish(e.Config.GraphiteHost)
  122. if err != nil {
  123. panic(err)
  124. }
  125. }
  126. // Retrieve CORS configuration
  127. corsInfo, err := ehttp.NewCORSInfo(e.Config.CorsOrigins)
  128. if err != nil {
  129. log.Fatal("CORS:", err)
  130. }
  131. // Create etcd key-value store and registry.
  132. e.Store = store.New()
  133. e.Registry = server.NewRegistry(e.Store)
  134. // Create stats objects
  135. followersStats := server.NewRaftFollowersStats(e.Config.Name)
  136. serverStats := server.NewRaftServerStats(e.Config.Name)
  137. // Calculate all of our timeouts
  138. heartbeatInterval := time.Duration(e.Config.Peer.HeartbeatInterval) * time.Millisecond
  139. electionTimeout := time.Duration(e.Config.Peer.ElectionTimeout) * time.Millisecond
  140. dialTimeout := (3 * heartbeatInterval) + electionTimeout
  141. responseHeaderTimeout := (3 * heartbeatInterval) + electionTimeout
  142. clientTransporter := &httpclient.Transport{
  143. ResponseHeaderTimeout: responseHeaderTimeout + extraTimeout,
  144. // This is a workaround for Transport.CancelRequest doesn't work on
  145. // HTTPS connections blocked. The patch for it is in progress,
  146. // and would be available in Go1.3
  147. // More: https://codereview.appspot.com/69280043/
  148. ConnectTimeout: dialTimeout + extraTimeout,
  149. RequestTimeout: responseHeaderTimeout + dialTimeout + 2*extraTimeout,
  150. }
  151. if e.Config.PeerTLSInfo().Scheme() == "https" {
  152. clientTLSConfig, err := e.Config.PeerTLSInfo().ClientConfig()
  153. if err != nil {
  154. log.Fatal("client TLS error: ", err)
  155. }
  156. clientTransporter.TLSClientConfig = clientTLSConfig
  157. clientTransporter.DisableCompression = true
  158. }
  159. client := server.NewClient(clientTransporter)
  160. // Create peer server
  161. psConfig := server.PeerServerConfig{
  162. Name: e.Config.Name,
  163. Scheme: e.Config.PeerTLSInfo().Scheme(),
  164. URL: e.Config.Peer.Addr,
  165. SnapshotCount: e.Config.SnapshotCount,
  166. RetryTimes: e.Config.MaxRetryAttempts,
  167. RetryInterval: e.Config.RetryInterval,
  168. }
  169. e.PeerServer = server.NewPeerServer(psConfig, client, e.Registry, e.Store, &mb, followersStats, serverStats)
  170. // Create raft transporter and server
  171. raftTransporter := server.NewTransporter(followersStats, serverStats, e.Registry, heartbeatInterval, dialTimeout, responseHeaderTimeout)
  172. if e.Config.PeerTLSInfo().Scheme() == "https" {
  173. raftClientTLSConfig, err := e.Config.PeerTLSInfo().ClientConfig()
  174. if err != nil {
  175. log.Fatal("raft client TLS error: ", err)
  176. }
  177. raftTransporter.SetTLSConfig(*raftClientTLSConfig)
  178. }
  179. raftServer, err := raft.NewServer(e.Config.Name, e.Config.DataDir, raftTransporter, e.Store, e.PeerServer, "")
  180. if err != nil {
  181. log.Fatal(err)
  182. }
  183. raftServer.SetElectionTimeout(electionTimeout)
  184. raftServer.SetHeartbeatInterval(heartbeatInterval)
  185. e.PeerServer.SetRaftServer(raftServer, e.Config.Snapshot)
  186. // Create etcd server
  187. e.Server = server.New(e.Config.Name, e.Config.Addr, e.PeerServer, e.Registry, e.Store, &mb)
  188. if e.Config.Trace() {
  189. e.Server.EnableTracing()
  190. }
  191. e.PeerServer.SetServer(e.Server)
  192. // Create standby server
  193. ssConfig := server.StandbyServerConfig{
  194. Name: e.Config.Name,
  195. PeerScheme: e.Config.PeerTLSInfo().Scheme(),
  196. PeerURL: e.Config.Peer.Addr,
  197. ClientURL: e.Config.Addr,
  198. DataDir: e.Config.DataDir,
  199. }
  200. e.StandbyServer = server.NewStandbyServer(ssConfig, client)
  201. e.StandbyServer.SetRaftServer(raftServer)
  202. // Generating config could be slow.
  203. // Put it here to make listen happen immediately after peer-server starting.
  204. peerTLSConfig := server.TLSServerConfig(e.Config.PeerTLSInfo())
  205. etcdTLSConfig := server.TLSServerConfig(e.Config.EtcdTLSInfo())
  206. if !e.StandbyServer.IsRunning() {
  207. startPeerServer, possiblePeers, err := e.PeerServer.FindCluster(e.Config.Discovery, e.Config.Peers)
  208. if err != nil {
  209. log.Fatal(err)
  210. }
  211. if startPeerServer {
  212. e.setMode(PeerMode)
  213. } else {
  214. e.StandbyServer.SyncCluster(possiblePeers)
  215. e.setMode(StandbyMode)
  216. }
  217. } else {
  218. e.setMode(StandbyMode)
  219. }
  220. serverHTTPHandler := &ehttp.CORSHandler{e.Server.HTTPHandler(), corsInfo}
  221. peerServerHTTPHandler := &ehttp.CORSHandler{e.PeerServer.HTTPHandler(), corsInfo}
  222. standbyServerHTTPHandler := &ehttp.CORSHandler{e.StandbyServer.ClientHTTPHandler(), corsInfo}
  223. log.Infof("etcd server [name %s, listen on %s, advertised url %s]", e.Server.Name, e.Config.BindAddr, e.Server.URL())
  224. listener := server.NewListener(e.Config.EtcdTLSInfo().Scheme(), e.Config.BindAddr, etcdTLSConfig)
  225. e.server = &http.Server{Handler: &ModeHandler{e, serverHTTPHandler, standbyServerHTTPHandler},
  226. ReadTimeout: time.Duration(e.Config.HTTPReadTimeout) * time.Second,
  227. WriteTimeout: time.Duration(e.Config.HTTPWriteTimeout) * time.Second,
  228. }
  229. log.Infof("peer server [name %s, listen on %s, advertised url %s]", e.PeerServer.Config.Name, e.Config.Peer.BindAddr, e.PeerServer.Config.URL)
  230. peerListener := server.NewListener(e.Config.PeerTLSInfo().Scheme(), e.Config.Peer.BindAddr, peerTLSConfig)
  231. e.peerServer = &http.Server{Handler: &ModeHandler{e, peerServerHTTPHandler, http.NotFoundHandler()},
  232. ReadTimeout: time.Duration(server.DefaultReadTimeout) * time.Second,
  233. WriteTimeout: time.Duration(server.DefaultWriteTimeout) * time.Second,
  234. }
  235. wg := sync.WaitGroup{}
  236. wg.Add(2)
  237. go func() {
  238. <-e.readyNotify
  239. defer wg.Done()
  240. if err := e.server.Serve(listener); err != nil {
  241. if !isListenerClosing(err) {
  242. log.Fatal(err)
  243. }
  244. }
  245. }()
  246. go func() {
  247. <-e.readyNotify
  248. defer wg.Done()
  249. if err := e.peerServer.Serve(peerListener); err != nil {
  250. if !isListenerClosing(err) {
  251. log.Fatal(err)
  252. }
  253. }
  254. }()
  255. e.runServer()
  256. listener.Close()
  257. peerListener.Close()
  258. wg.Wait()
  259. log.Infof("etcd instance is stopped [name %s]", e.Config.Name)
  260. close(e.stopNotify)
  261. }
  262. func (e *Etcd) runServer() {
  263. var removeNotify <-chan bool
  264. for {
  265. if e.mode == PeerMode {
  266. log.Infof("%v starting in peer mode", e.Config.Name)
  267. // Starting peer server should be followed close by listening on its port
  268. // If not, it may leave many requests unaccepted, or cannot receive heartbeat from the cluster.
  269. // One severe problem caused if failing receiving heartbeats is when the second node joins one-node cluster,
  270. // the cluster could be out of work as long as the two nodes cannot transfer messages.
  271. e.PeerServer.Start(e.Config.Snapshot, e.Config.ClusterConfig())
  272. removeNotify = e.PeerServer.RemoveNotify()
  273. } else {
  274. log.Infof("%v starting in standby mode", e.Config.Name)
  275. e.StandbyServer.Start()
  276. removeNotify = e.StandbyServer.RemoveNotify()
  277. }
  278. // etcd server is ready to accept connections, notify waiters.
  279. e.onceReady.Do(func() { close(e.readyNotify) })
  280. select {
  281. case <-e.closeChan:
  282. e.PeerServer.Stop()
  283. e.StandbyServer.Stop()
  284. return
  285. case <-removeNotify:
  286. }
  287. if e.mode == PeerMode {
  288. peerURLs := e.Registry.PeerURLs(e.PeerServer.RaftServer().Leader(), e.Config.Name)
  289. e.StandbyServer.SyncCluster(peerURLs)
  290. e.setMode(StandbyMode)
  291. } else {
  292. // Create etcd key-value store and registry.
  293. e.Store = store.New()
  294. e.Registry = server.NewRegistry(e.Store)
  295. e.PeerServer.SetStore(e.Store)
  296. e.PeerServer.SetRegistry(e.Registry)
  297. e.Server.SetStore(e.Store)
  298. e.Server.SetRegistry(e.Registry)
  299. // Generate new peer server here.
  300. // TODO(yichengq): raft server cannot be started after stopped.
  301. // It should be removed when raft restart is implemented.
  302. heartbeatInterval := time.Duration(e.Config.Peer.HeartbeatInterval) * time.Millisecond
  303. electionTimeout := time.Duration(e.Config.Peer.ElectionTimeout) * time.Millisecond
  304. raftServer, err := raft.NewServer(e.Config.Name, e.Config.DataDir, e.PeerServer.RaftServer().Transporter(), e.Store, e.PeerServer, "")
  305. if err != nil {
  306. log.Fatal(err)
  307. }
  308. raftServer.SetElectionTimeout(electionTimeout)
  309. raftServer.SetHeartbeatInterval(heartbeatInterval)
  310. e.PeerServer.SetRaftServer(raftServer, e.Config.Snapshot)
  311. e.StandbyServer.SetRaftServer(raftServer)
  312. e.PeerServer.SetJoinIndex(e.StandbyServer.JoinIndex())
  313. e.setMode(PeerMode)
  314. }
  315. }
  316. }
  317. // Stop the etcd instance.
  318. func (e *Etcd) Stop() {
  319. close(e.closeChan)
  320. <-e.stopNotify
  321. }
  322. // ReadyNotify returns a channel that is going to be closed
  323. // when the etcd instance is ready to accept connections.
  324. func (e *Etcd) ReadyNotify() <-chan bool {
  325. return e.readyNotify
  326. }
  327. func (e *Etcd) Mode() Mode {
  328. e.modeMutex.Lock()
  329. defer e.modeMutex.Unlock()
  330. return e.mode
  331. }
  332. func (e *Etcd) setMode(m Mode) {
  333. e.modeMutex.Lock()
  334. defer e.modeMutex.Unlock()
  335. e.mode = m
  336. }
  337. func isListenerClosing(err error) bool {
  338. // An error string equivalent to net.errClosing for using with
  339. // http.Serve() during server shutdown. Need to re-declare
  340. // here because it is not exported by "net" package.
  341. const errClosing = "use of closed network connection"
  342. return strings.Contains(err.Error(), errClosing)
  343. }
  344. type ModeGetter interface {
  345. Mode() Mode
  346. }
  347. type ModeHandler struct {
  348. ModeGetter
  349. PeerModeHandler http.Handler
  350. StandbyModeHandler http.Handler
  351. }
  352. func (h *ModeHandler) ServeHTTP(w http.ResponseWriter, r *http.Request) {
  353. switch h.Mode() {
  354. case PeerMode:
  355. h.PeerModeHandler.ServeHTTP(w, r)
  356. case StandbyMode:
  357. h.StandbyModeHandler.ServeHTTP(w, r)
  358. }
  359. }
  360. type Mode int
  361. const (
  362. PeerMode Mode = iota
  363. StandbyMode
  364. )