lease.go 9.4 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433
  1. // Copyright 2016 CoreOS, Inc.
  2. //
  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. //
  7. // http://www.apache.org/licenses/LICENSE-2.0
  8. //
  9. // Unless required by applicable law or agreed to in writing, software
  10. // distributed under the License is distributed on an "AS IS" BASIS,
  11. // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  12. // See the License for the specific language governing permissions and
  13. // limitations under the License.
  14. package clientv3
  15. import (
  16. "sync"
  17. "time"
  18. "github.com/coreos/etcd/Godeps/_workspace/src/golang.org/x/net/context"
  19. "github.com/coreos/etcd/Godeps/_workspace/src/google.golang.org/grpc"
  20. pb "github.com/coreos/etcd/etcdserver/etcdserverpb"
  21. "github.com/coreos/etcd/lease"
  22. )
  23. type (
  24. LeaseCreateResponse pb.LeaseCreateResponse
  25. LeaseRevokeResponse pb.LeaseRevokeResponse
  26. LeaseKeepAliveResponse pb.LeaseKeepAliveResponse
  27. )
  28. const (
  29. // a small buffer to store unsent lease responses.
  30. leaseResponseChSize = 16
  31. )
  32. type Lease interface {
  33. // Create creates a new lease.
  34. Create(ctx context.Context, ttl int64) (*LeaseCreateResponse, error)
  35. // Revoke revokes the given lease.
  36. Revoke(ctx context.Context, id lease.LeaseID) (*LeaseRevokeResponse, error)
  37. // KeepAlive keeps the given lease alive forever.
  38. KeepAlive(ctx context.Context, id lease.LeaseID) (<-chan *LeaseKeepAliveResponse, error)
  39. // KeepAliveOnce renews the lease once. In most of the cases, Keepalive
  40. // should be used instead of KeepAliveOnce.
  41. KeepAliveOnce(ctx context.Context, id lease.LeaseID) (*LeaseKeepAliveResponse, error)
  42. // Close releases all resources Lease keeps for efficient communication
  43. // with the etcd server.
  44. Close() error
  45. }
  46. type lessor struct {
  47. c *Client
  48. mu sync.Mutex // guards all fields
  49. conn *grpc.ClientConn // conn in-use
  50. // donec is closed when recvKeepAliveLoop stops
  51. donec chan struct{}
  52. remote pb.LeaseClient
  53. stream pb.Lease_LeaseKeepAliveClient
  54. streamCancel context.CancelFunc
  55. stopCtx context.Context
  56. stopCancel context.CancelFunc
  57. keepAlives map[lease.LeaseID]*keepAlive
  58. }
  59. // keepAlive multiplexes a keepalive for a lease over multiple channels
  60. type keepAlive struct {
  61. chs []chan<- *LeaseKeepAliveResponse
  62. ctxs []context.Context
  63. // deadline is the next time to send a keep alive message
  64. deadline time.Time
  65. // donec is closed on lease revoke, expiration, or cancel.
  66. donec chan struct{}
  67. }
  68. func NewLease(c *Client) Lease {
  69. l := &lessor{
  70. c: c,
  71. conn: c.ActiveConnection(),
  72. donec: make(chan struct{}),
  73. keepAlives: make(map[lease.LeaseID]*keepAlive),
  74. }
  75. l.remote = pb.NewLeaseClient(l.conn)
  76. l.stopCtx, l.stopCancel = context.WithCancel(context.Background())
  77. go l.recvKeepAliveLoop()
  78. return l
  79. }
  80. func (l *lessor) Create(ctx context.Context, ttl int64) (*LeaseCreateResponse, error) {
  81. cctx, cancel := context.WithCancel(ctx)
  82. done := cancelWhenStop(cancel, l.stopCtx.Done())
  83. defer close(done)
  84. for {
  85. r := &pb.LeaseCreateRequest{TTL: ttl}
  86. resp, err := l.getRemote().LeaseCreate(cctx, r)
  87. if err == nil {
  88. return (*LeaseCreateResponse)(resp), nil
  89. }
  90. if isRPCError(err) {
  91. return nil, err
  92. }
  93. if nerr := l.switchRemoteAndStream(err); nerr != nil {
  94. return nil, nerr
  95. }
  96. }
  97. }
  98. func (l *lessor) Revoke(ctx context.Context, id lease.LeaseID) (*LeaseRevokeResponse, error) {
  99. cctx, cancel := context.WithCancel(ctx)
  100. done := cancelWhenStop(cancel, l.stopCtx.Done())
  101. defer close(done)
  102. for {
  103. r := &pb.LeaseRevokeRequest{ID: int64(id)}
  104. resp, err := l.getRemote().LeaseRevoke(cctx, r)
  105. if err == nil {
  106. return (*LeaseRevokeResponse)(resp), nil
  107. }
  108. if isRPCError(err) {
  109. return nil, err
  110. }
  111. if nerr := l.switchRemoteAndStream(err); nerr != nil {
  112. return nil, nerr
  113. }
  114. }
  115. }
  116. func (l *lessor) KeepAlive(ctx context.Context, id lease.LeaseID) (<-chan *LeaseKeepAliveResponse, error) {
  117. ch := make(chan *LeaseKeepAliveResponse, leaseResponseChSize)
  118. l.mu.Lock()
  119. ka, ok := l.keepAlives[id]
  120. if !ok {
  121. // create fresh keep alive
  122. ka = &keepAlive{
  123. chs: []chan<- *LeaseKeepAliveResponse{ch},
  124. ctxs: []context.Context{ctx},
  125. deadline: time.Now(),
  126. donec: make(chan struct{}),
  127. }
  128. l.keepAlives[id] = ka
  129. } else {
  130. // add channel and context to existing keep alive
  131. ka.ctxs = append(ka.ctxs, ctx)
  132. ka.chs = append(ka.chs, ch)
  133. }
  134. l.mu.Unlock()
  135. go l.keepAliveCtxCloser(id, ctx, ka.donec)
  136. return ch, nil
  137. }
  138. func (l *lessor) KeepAliveOnce(ctx context.Context, id lease.LeaseID) (*LeaseKeepAliveResponse, error) {
  139. cctx, cancel := context.WithCancel(ctx)
  140. done := cancelWhenStop(cancel, l.stopCtx.Done())
  141. defer close(done)
  142. for {
  143. resp, err := l.keepAliveOnce(cctx, id)
  144. if err == nil {
  145. return resp, err
  146. }
  147. nerr := l.switchRemoteAndStream(err)
  148. if nerr != nil {
  149. return nil, nerr
  150. }
  151. }
  152. }
  153. func (l *lessor) Close() error {
  154. l.stopCancel()
  155. <-l.donec
  156. return nil
  157. }
  158. func (l *lessor) keepAliveCtxCloser(id lease.LeaseID, ctx context.Context, donec <-chan struct{}) {
  159. select {
  160. case <-donec:
  161. return
  162. case <-l.donec:
  163. return
  164. case <-ctx.Done():
  165. }
  166. l.mu.Lock()
  167. defer l.mu.Unlock()
  168. ka, ok := l.keepAlives[id]
  169. if !ok {
  170. return
  171. }
  172. // close channel and remove context if still associated with keep alive
  173. for i, c := range ka.ctxs {
  174. if c == ctx {
  175. close(ka.chs[i])
  176. ka.ctxs = append(ka.ctxs[:i], ka.ctxs[i+1:]...)
  177. ka.chs = append(ka.chs[:i], ka.chs[i+1:]...)
  178. break
  179. }
  180. }
  181. // remove if no one more listeners
  182. if len(ka.chs) == 0 {
  183. delete(l.keepAlives, id)
  184. }
  185. }
  186. func (l *lessor) keepAliveOnce(ctx context.Context, id lease.LeaseID) (*LeaseKeepAliveResponse, error) {
  187. stream, err := l.getRemote().LeaseKeepAlive(ctx)
  188. if err != nil {
  189. return nil, err
  190. }
  191. err = stream.Send(&pb.LeaseKeepAliveRequest{ID: int64(id)})
  192. if err != nil {
  193. return nil, err
  194. }
  195. resp, rerr := stream.Recv()
  196. if rerr != nil {
  197. return nil, rerr
  198. }
  199. return (*LeaseKeepAliveResponse)(resp), nil
  200. }
  201. func (l *lessor) recvKeepAliveLoop() {
  202. defer func() {
  203. l.stopCancel()
  204. l.mu.Lock()
  205. close(l.donec)
  206. for _, ka := range l.keepAlives {
  207. ka.Close()
  208. }
  209. l.keepAlives = make(map[lease.LeaseID]*keepAlive)
  210. l.mu.Unlock()
  211. }()
  212. stream, serr := l.resetRecv()
  213. for serr == nil {
  214. resp, err := stream.Recv()
  215. if err != nil {
  216. if isRPCError(err) {
  217. return
  218. }
  219. stream, serr = l.resetRecv()
  220. continue
  221. }
  222. l.recvKeepAlive(resp)
  223. }
  224. }
  225. // resetRecv opens a new lease stream and starts sending LeaseKeepAliveRequests
  226. func (l *lessor) resetRecv() (pb.Lease_LeaseKeepAliveClient, error) {
  227. if err := l.switchRemoteAndStream(nil); err != nil {
  228. return nil, err
  229. }
  230. stream := l.getKeepAliveStream()
  231. go l.sendKeepAliveLoop(stream)
  232. return stream, nil
  233. }
  234. // recvKeepAlive updates a lease based on its LeaseKeepAliveResponse
  235. func (l *lessor) recvKeepAlive(resp *pb.LeaseKeepAliveResponse) {
  236. id := lease.LeaseID(resp.ID)
  237. l.mu.Lock()
  238. defer l.mu.Unlock()
  239. ka, ok := l.keepAlives[id]
  240. if !ok {
  241. return
  242. }
  243. if resp.TTL <= 0 {
  244. // lease expired; close all keep alive channels
  245. delete(l.keepAlives, id)
  246. ka.Close()
  247. return
  248. }
  249. // send update to all channels
  250. nextDeadline := time.Now().Add(1 + time.Duration(resp.TTL/3)*time.Second)
  251. for _, ch := range ka.chs {
  252. select {
  253. case ch <- (*LeaseKeepAliveResponse)(resp):
  254. ka.deadline = nextDeadline
  255. default:
  256. }
  257. }
  258. }
  259. // sendKeepAliveLoop sends LeaseKeepAliveRequests for the lifetime of a lease stream
  260. func (l *lessor) sendKeepAliveLoop(stream pb.Lease_LeaseKeepAliveClient) {
  261. for {
  262. select {
  263. case <-time.After(500 * time.Millisecond):
  264. case <-l.donec:
  265. return
  266. case <-l.stopCtx.Done():
  267. return
  268. }
  269. tosend := make([]lease.LeaseID, 0)
  270. now := time.Now()
  271. l.mu.Lock()
  272. for id, ka := range l.keepAlives {
  273. if ka.deadline.Before(now) {
  274. tosend = append(tosend, id)
  275. }
  276. }
  277. l.mu.Unlock()
  278. for _, id := range tosend {
  279. r := &pb.LeaseKeepAliveRequest{ID: int64(id)}
  280. if err := stream.Send(r); err != nil {
  281. // TODO do something with this error?
  282. return
  283. }
  284. }
  285. }
  286. }
  287. func (l *lessor) getRemote() pb.LeaseClient {
  288. l.mu.Lock()
  289. defer l.mu.Unlock()
  290. return l.remote
  291. }
  292. func (l *lessor) getKeepAliveStream() pb.Lease_LeaseKeepAliveClient {
  293. l.mu.Lock()
  294. defer l.mu.Unlock()
  295. return l.stream
  296. }
  297. func (l *lessor) switchRemoteAndStream(prevErr error) error {
  298. l.mu.Lock()
  299. conn := l.conn
  300. l.mu.Unlock()
  301. var (
  302. err error
  303. newConn *grpc.ClientConn
  304. )
  305. if prevErr != nil {
  306. conn.Close()
  307. newConn, err = l.c.retryConnection(conn, prevErr)
  308. if err != nil {
  309. return err
  310. }
  311. }
  312. l.mu.Lock()
  313. if newConn != nil {
  314. l.conn = newConn
  315. }
  316. l.remote = pb.NewLeaseClient(l.conn)
  317. l.mu.Unlock()
  318. serr := l.newStream()
  319. if serr != nil {
  320. return serr
  321. }
  322. return nil
  323. }
  324. func (l *lessor) newStream() error {
  325. sctx, cancel := context.WithCancel(l.stopCtx)
  326. stream, err := l.getRemote().LeaseKeepAlive(sctx)
  327. if err != nil {
  328. cancel()
  329. return err
  330. }
  331. l.mu.Lock()
  332. defer l.mu.Unlock()
  333. if l.stream != nil && l.streamCancel != nil {
  334. l.stream.CloseSend()
  335. l.streamCancel()
  336. }
  337. l.streamCancel = cancel
  338. l.stream = stream
  339. return nil
  340. }
  341. func (ka *keepAlive) Close() {
  342. close(ka.donec)
  343. for _, ch := range ka.chs {
  344. close(ch)
  345. }
  346. }
  347. // cancelWhenStop calls cancel when the given stopc fires. It returns a done chan. done
  348. // should be closed when the work is finished. When done fires, cancelWhenStop will release
  349. // its internal resource.
  350. func cancelWhenStop(cancel context.CancelFunc, stopc <-chan struct{}) chan<- struct{} {
  351. done := make(chan struct{}, 1)
  352. go func() {
  353. select {
  354. case <-stopc:
  355. case <-done:
  356. }
  357. cancel()
  358. }()
  359. return done
  360. }