lease.go 10 KB

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