http.go 13 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479
  1. package etcdhttp
  2. import (
  3. "encoding/json"
  4. "errors"
  5. "fmt"
  6. "io/ioutil"
  7. "log"
  8. "net/http"
  9. "net/url"
  10. "strconv"
  11. "strings"
  12. "time"
  13. "github.com/coreos/etcd/Godeps/_workspace/src/code.google.com/p/go.net/context"
  14. "github.com/coreos/etcd/Godeps/_workspace/src/github.com/jonboulle/clockwork"
  15. etcdErr "github.com/coreos/etcd/error"
  16. "github.com/coreos/etcd/etcdserver"
  17. "github.com/coreos/etcd/etcdserver/etcdserverpb"
  18. "github.com/coreos/etcd/raft/raftpb"
  19. "github.com/coreos/etcd/store"
  20. )
  21. const (
  22. keysPrefix = "/v2/keys"
  23. deprecatedMachinesPrefix = "/v2/machines"
  24. adminMembersPrefix = "/v2/admin/members/"
  25. raftPrefix = "/raft"
  26. statsPrefix = "/v2/stats"
  27. // time to wait for response from EtcdServer requests
  28. defaultServerTimeout = 500 * time.Millisecond
  29. // time to wait for a Watch request
  30. defaultWatchTimeout = 5 * time.Minute
  31. )
  32. var errClosed = errors.New("etcdhttp: client closed connection")
  33. // NewClientHandler generates a muxed http.Handler with the given parameters to serve etcd client requests.
  34. func NewClientHandler(server *etcdserver.EtcdServer) http.Handler {
  35. sh := &serverHandler{
  36. server: server,
  37. clusterStore: server.ClusterStore,
  38. stats: server,
  39. timer: server,
  40. timeout: defaultServerTimeout,
  41. }
  42. mux := http.NewServeMux()
  43. mux.HandleFunc(keysPrefix, sh.serveKeys)
  44. mux.HandleFunc(keysPrefix+"/", sh.serveKeys)
  45. mux.HandleFunc(statsPrefix+"/store", sh.serveStoreStats)
  46. mux.HandleFunc(statsPrefix+"/self", sh.serveSelfStats)
  47. mux.HandleFunc(statsPrefix+"/leader", sh.serveLeaderStats)
  48. // TODO: dynamic configuration may make this outdated. take care of it.
  49. // TODO: dynamic configuration may introduce race also.
  50. // TODO: add serveMembers
  51. mux.HandleFunc(deprecatedMachinesPrefix, sh.serveMachines)
  52. mux.HandleFunc(adminMembersPrefix, sh.serveAdminMembers)
  53. mux.HandleFunc("/", http.NotFound)
  54. return mux
  55. }
  56. // NewPeerHandler generates an http.Handler to handle etcd peer (raft) requests.
  57. func NewPeerHandler(server *etcdserver.EtcdServer) http.Handler {
  58. sh := &serverHandler{
  59. server: server,
  60. stats: server,
  61. }
  62. mux := http.NewServeMux()
  63. mux.HandleFunc(raftPrefix, sh.serveRaft)
  64. mux.HandleFunc("/", http.NotFound)
  65. return mux
  66. }
  67. // serverHandler provides http.Handlers for etcd client and raft communication.
  68. type serverHandler struct {
  69. timeout time.Duration
  70. server etcdserver.Server
  71. stats etcdserver.Stats
  72. timer etcdserver.RaftTimer
  73. clusterStore etcdserver.ClusterStore
  74. }
  75. func (h serverHandler) serveKeys(w http.ResponseWriter, r *http.Request) {
  76. if !allowMethod(w, r.Method, "GET", "PUT", "POST", "DELETE") {
  77. return
  78. }
  79. ctx, cancel := context.WithTimeout(context.Background(), h.timeout)
  80. defer cancel()
  81. rr, err := parseRequest(r, etcdserver.GenID(), clockwork.NewRealClock())
  82. if err != nil {
  83. writeError(w, err)
  84. return
  85. }
  86. resp, err := h.server.Do(ctx, rr)
  87. if err != nil {
  88. writeError(w, err)
  89. return
  90. }
  91. switch {
  92. case resp.Event != nil:
  93. if err := writeEvent(w, resp.Event, h.timer); err != nil {
  94. // Should never be reached
  95. log.Printf("error writing event: %v", err)
  96. }
  97. case resp.Watcher != nil:
  98. ctx, cancel := context.WithTimeout(context.Background(), defaultWatchTimeout)
  99. defer cancel()
  100. handleWatch(ctx, w, resp.Watcher, rr.Stream, h.timer)
  101. default:
  102. writeError(w, errors.New("received response with no Event/Watcher!"))
  103. }
  104. }
  105. // serveMachines responds address list in the format '0.0.0.0, 1.1.1.1'.
  106. func (h serverHandler) serveMachines(w http.ResponseWriter, r *http.Request) {
  107. if !allowMethod(w, r.Method, "GET", "HEAD") {
  108. return
  109. }
  110. endpoints := h.clusterStore.Get().ClientURLs()
  111. w.Write([]byte(strings.Join(endpoints, ", ")))
  112. }
  113. func (h serverHandler) serveAdminMembers(w http.ResponseWriter, r *http.Request) {
  114. if !allowMethod(w, r.Method, "PUT", "DELETE") {
  115. return
  116. }
  117. ctx, cancel := context.WithTimeout(context.Background(), defaultServerTimeout)
  118. defer cancel()
  119. idStr := strings.TrimPrefix(r.URL.Path, adminMembersPrefix)
  120. id, err := strconv.ParseUint(idStr, 16, 64)
  121. if err != nil {
  122. http.Error(w, err.Error(), http.StatusBadRequest)
  123. return
  124. }
  125. switch r.Method {
  126. case "PUT":
  127. if err := r.ParseForm(); err != nil {
  128. http.Error(w, err.Error(), http.StatusBadRequest)
  129. return
  130. }
  131. peerURLs := r.PostForm["PeerURLs"]
  132. log.Printf("etcdhttp: add node %x with peer urls %v", id, peerURLs)
  133. m := etcdserver.Member{
  134. ID: id,
  135. RaftAttributes: etcdserver.RaftAttributes{
  136. PeerURLs: peerURLs,
  137. },
  138. }
  139. if err := h.server.AddMember(ctx, m); err != nil {
  140. log.Printf("etcdhttp: error adding node %x: %v", id, err)
  141. writeError(w, err)
  142. return
  143. }
  144. w.WriteHeader(http.StatusCreated)
  145. case "DELETE":
  146. log.Printf("etcdhttp: remove node %x", id)
  147. if err := h.server.RemoveMember(ctx, id); err != nil {
  148. log.Printf("etcdhttp: error removing node %x: %v", id, err)
  149. writeError(w, err)
  150. return
  151. }
  152. w.WriteHeader(http.StatusNoContent)
  153. }
  154. }
  155. func (h serverHandler) serveStoreStats(w http.ResponseWriter, r *http.Request) {
  156. if !allowMethod(w, r.Method, "GET") {
  157. return
  158. }
  159. w.Header().Set("Content-Type", "application/json")
  160. w.Write(h.stats.StoreStats())
  161. }
  162. func (h serverHandler) serveSelfStats(w http.ResponseWriter, r *http.Request) {
  163. if !allowMethod(w, r.Method, "GET") {
  164. return
  165. }
  166. w.Header().Set("Content-Type", "application/json")
  167. w.Write(h.stats.SelfStats())
  168. }
  169. func (h serverHandler) serveLeaderStats(w http.ResponseWriter, r *http.Request) {
  170. if !allowMethod(w, r.Method, "GET") {
  171. return
  172. }
  173. w.Header().Set("Content-Type", "application/json")
  174. w.Write(h.stats.LeaderStats())
  175. }
  176. func (h serverHandler) serveRaft(w http.ResponseWriter, r *http.Request) {
  177. if !allowMethod(w, r.Method, "POST") {
  178. return
  179. }
  180. b, err := ioutil.ReadAll(r.Body)
  181. if err != nil {
  182. log.Println("etcdhttp: error reading raft message:", err)
  183. http.Error(w, "error reading raft message", http.StatusBadRequest)
  184. return
  185. }
  186. var m raftpb.Message
  187. if err := m.Unmarshal(b); err != nil {
  188. log.Println("etcdhttp: error unmarshaling raft message:", err)
  189. http.Error(w, "error unmarshaling raft message", http.StatusBadRequest)
  190. return
  191. }
  192. log.Printf("etcdhttp: raft recv message from %#x: %+v", m.From, m)
  193. if m.Type == raftpb.MsgApp {
  194. h.stats.UpdateRecvApp(m.From, r.ContentLength)
  195. }
  196. if err := h.server.Process(context.TODO(), m); err != nil {
  197. log.Println("etcdhttp: error processing raft message:", err)
  198. writeError(w, err)
  199. return
  200. }
  201. w.WriteHeader(http.StatusNoContent)
  202. }
  203. // parseRequest converts a received http.Request to a server Request,
  204. // performing validation of supplied fields as appropriate.
  205. // If any validation fails, an empty Request and non-nil error is returned.
  206. func parseRequest(r *http.Request, id uint64, clock clockwork.Clock) (etcdserverpb.Request, error) {
  207. emptyReq := etcdserverpb.Request{}
  208. err := r.ParseForm()
  209. if err != nil {
  210. return emptyReq, etcdErr.NewRequestError(
  211. etcdErr.EcodeInvalidForm,
  212. err.Error(),
  213. )
  214. }
  215. if !strings.HasPrefix(r.URL.Path, keysPrefix) {
  216. return emptyReq, etcdErr.NewRequestError(
  217. etcdErr.EcodeInvalidForm,
  218. "incorrect key prefix",
  219. )
  220. }
  221. p := r.URL.Path[len(keysPrefix):]
  222. var pIdx, wIdx uint64
  223. if pIdx, err = getUint64(r.Form, "prevIndex"); err != nil {
  224. return emptyReq, etcdErr.NewRequestError(
  225. etcdErr.EcodeIndexNaN,
  226. `invalid value for "prevIndex"`,
  227. )
  228. }
  229. if wIdx, err = getUint64(r.Form, "waitIndex"); err != nil {
  230. return emptyReq, etcdErr.NewRequestError(
  231. etcdErr.EcodeIndexNaN,
  232. `invalid value for "waitIndex"`,
  233. )
  234. }
  235. var rec, sort, wait, dir, stream bool
  236. if rec, err = getBool(r.Form, "recursive"); err != nil {
  237. return emptyReq, etcdErr.NewRequestError(
  238. etcdErr.EcodeInvalidField,
  239. `invalid value for "recursive"`,
  240. )
  241. }
  242. if sort, err = getBool(r.Form, "sorted"); err != nil {
  243. return emptyReq, etcdErr.NewRequestError(
  244. etcdErr.EcodeInvalidField,
  245. `invalid value for "sorted"`,
  246. )
  247. }
  248. if wait, err = getBool(r.Form, "wait"); err != nil {
  249. return emptyReq, etcdErr.NewRequestError(
  250. etcdErr.EcodeInvalidField,
  251. `invalid value for "wait"`,
  252. )
  253. }
  254. // TODO(jonboulle): define what parameters dir is/isn't compatible with?
  255. if dir, err = getBool(r.Form, "dir"); err != nil {
  256. return emptyReq, etcdErr.NewRequestError(
  257. etcdErr.EcodeInvalidField,
  258. `invalid value for "dir"`,
  259. )
  260. }
  261. if stream, err = getBool(r.Form, "stream"); err != nil {
  262. return emptyReq, etcdErr.NewRequestError(
  263. etcdErr.EcodeInvalidField,
  264. `invalid value for "stream"`,
  265. )
  266. }
  267. if wait && r.Method != "GET" {
  268. return emptyReq, etcdErr.NewRequestError(
  269. etcdErr.EcodeInvalidField,
  270. `"wait" can only be used with GET requests`,
  271. )
  272. }
  273. pV := r.FormValue("prevValue")
  274. if _, ok := r.Form["prevValue"]; ok && pV == "" {
  275. return emptyReq, etcdErr.NewRequestError(
  276. etcdErr.EcodeInvalidField,
  277. `"prevValue" cannot be empty`,
  278. )
  279. }
  280. // TTL is nullable, so leave it null if not specified
  281. // or an empty string
  282. var ttl *uint64
  283. if len(r.FormValue("ttl")) > 0 {
  284. i, err := getUint64(r.Form, "ttl")
  285. if err != nil {
  286. return emptyReq, etcdErr.NewRequestError(
  287. etcdErr.EcodeTTLNaN,
  288. `invalid value for "ttl"`,
  289. )
  290. }
  291. ttl = &i
  292. }
  293. // prevExist is nullable, so leave it null if not specified
  294. var pe *bool
  295. if _, ok := r.Form["prevExist"]; ok {
  296. bv, err := getBool(r.Form, "prevExist")
  297. if err != nil {
  298. return emptyReq, etcdErr.NewRequestError(
  299. etcdErr.EcodeInvalidField,
  300. "invalid value for prevExist",
  301. )
  302. }
  303. pe = &bv
  304. }
  305. rr := etcdserverpb.Request{
  306. ID: id,
  307. Method: r.Method,
  308. Path: p,
  309. Val: r.FormValue("value"),
  310. Dir: dir,
  311. PrevValue: pV,
  312. PrevIndex: pIdx,
  313. PrevExist: pe,
  314. Recursive: rec,
  315. Since: wIdx,
  316. Sorted: sort,
  317. Stream: stream,
  318. Wait: wait,
  319. }
  320. if pe != nil {
  321. rr.PrevExist = pe
  322. }
  323. // Null TTL is equivalent to unset Expiration
  324. if ttl != nil {
  325. expr := time.Duration(*ttl) * time.Second
  326. rr.Expiration = clock.Now().Add(expr).UnixNano()
  327. }
  328. return rr, nil
  329. }
  330. // getUint64 extracts a uint64 by the given key from a Form. If the key does
  331. // not exist in the form, 0 is returned. If the key exists but the value is
  332. // badly formed, an error is returned. If multiple values are present only the
  333. // first is considered.
  334. func getUint64(form url.Values, key string) (i uint64, err error) {
  335. if vals, ok := form[key]; ok {
  336. i, err = strconv.ParseUint(vals[0], 10, 64)
  337. }
  338. return
  339. }
  340. // getBool extracts a bool by the given key from a Form. If the key does not
  341. // exist in the form, false is returned. If the key exists but the value is
  342. // badly formed, an error is returned. If multiple values are present only the
  343. // first is considered.
  344. func getBool(form url.Values, key string) (b bool, err error) {
  345. if vals, ok := form[key]; ok {
  346. b, err = strconv.ParseBool(vals[0])
  347. }
  348. return
  349. }
  350. // writeError logs and writes the given Error to the ResponseWriter
  351. // If Error is an etcdErr, it is rendered to the ResponseWriter
  352. // Otherwise, it is assumed to be an InternalServerError
  353. func writeError(w http.ResponseWriter, err error) {
  354. if err == nil {
  355. return
  356. }
  357. log.Println(err)
  358. if e, ok := err.(*etcdErr.Error); ok {
  359. e.Write(w)
  360. } else {
  361. http.Error(w, "Internal Server Error", http.StatusInternalServerError)
  362. }
  363. }
  364. // writeEvent serializes a single Event and writes the resulting
  365. // JSON to the given ResponseWriter, along with the appropriate
  366. // headers
  367. func writeEvent(w http.ResponseWriter, ev *store.Event, rt etcdserver.RaftTimer) error {
  368. if ev == nil {
  369. return errors.New("cannot write empty Event!")
  370. }
  371. w.Header().Set("Content-Type", "application/json")
  372. w.Header().Set("X-Etcd-Index", fmt.Sprint(ev.EtcdIndex))
  373. w.Header().Set("X-Raft-Index", fmt.Sprint(rt.Index()))
  374. w.Header().Set("X-Raft-Term", fmt.Sprint(rt.Term()))
  375. if ev.IsCreated() {
  376. w.WriteHeader(http.StatusCreated)
  377. }
  378. return json.NewEncoder(w).Encode(ev)
  379. }
  380. func handleWatch(ctx context.Context, w http.ResponseWriter, wa store.Watcher, stream bool, rt etcdserver.RaftTimer) {
  381. defer wa.Remove()
  382. ech := wa.EventChan()
  383. var nch <-chan bool
  384. if x, ok := w.(http.CloseNotifier); ok {
  385. nch = x.CloseNotify()
  386. }
  387. w.Header().Set("Content-Type", "application/json")
  388. w.Header().Set("X-Etcd-Index", fmt.Sprint(wa.StartIndex()))
  389. w.Header().Set("X-Raft-Index", fmt.Sprint(rt.Index()))
  390. w.Header().Set("X-Raft-Term", fmt.Sprint(rt.Term()))
  391. w.WriteHeader(http.StatusOK)
  392. // Ensure headers are flushed early, in case of long polling
  393. w.(http.Flusher).Flush()
  394. for {
  395. select {
  396. case <-nch:
  397. // Client closed connection. Nothing to do.
  398. return
  399. case <-ctx.Done():
  400. // Timed out. net/http will close the connection for us, so nothing to do.
  401. return
  402. case ev, ok := <-ech:
  403. if !ok {
  404. // If the channel is closed this may be an indication of
  405. // that notifications are much more than we are able to
  406. // send to the client in time. Then we simply end streaming.
  407. return
  408. }
  409. if err := json.NewEncoder(w).Encode(ev); err != nil {
  410. // Should never be reached
  411. log.Println("error writing event: %v", err)
  412. return
  413. }
  414. if !stream {
  415. return
  416. }
  417. w.(http.Flusher).Flush()
  418. }
  419. }
  420. }
  421. // allowMethod verifies that the given method is one of the allowed methods,
  422. // and if not, it writes an error to w. A boolean is returned indicating
  423. // whether or not the method is allowed.
  424. func allowMethod(w http.ResponseWriter, m string, ms ...string) bool {
  425. for _, meth := range ms {
  426. if m == meth {
  427. return true
  428. }
  429. }
  430. w.Header().Set("Allow", strings.Join(ms, ","))
  431. http.Error(w, "Method Not Allowed", http.StatusMethodNotAllowed)
  432. return false
  433. }