lease.go 9.5 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422
  1. // Copyright 2016 The etcd Authors
  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/etcdserver/api/v3rpc/rpctypes"
  19. pb "github.com/coreos/etcd/etcdserver/etcdserverpb"
  20. "golang.org/x/net/context"
  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. mu sync.Mutex // guards all fields
  61. // donec is closed when recvKeepAliveLoop stops
  62. donec chan struct{}
  63. remote pb.LeaseClient
  64. stream pb.Lease_LeaseKeepAliveClient
  65. streamCancel context.CancelFunc
  66. stopCtx context.Context
  67. stopCancel context.CancelFunc
  68. keepAlives map[LeaseID]*keepAlive
  69. }
  70. // keepAlive multiplexes a keepalive for a lease over multiple channels
  71. type keepAlive struct {
  72. chs []chan<- *LeaseKeepAliveResponse
  73. ctxs []context.Context
  74. // deadline is the next time to send a keep alive message
  75. deadline time.Time
  76. // donec is closed on lease revoke, expiration, or cancel.
  77. donec chan struct{}
  78. }
  79. func NewLease(c *Client) Lease {
  80. l := &lessor{
  81. donec: make(chan struct{}),
  82. keepAlives: make(map[LeaseID]*keepAlive),
  83. remote: pb.NewLeaseClient(c.conn),
  84. }
  85. l.stopCtx, l.stopCancel = context.WithCancel(context.Background())
  86. go l.recvKeepAliveLoop()
  87. return l
  88. }
  89. func (l *lessor) Grant(ctx context.Context, ttl int64) (*LeaseGrantResponse, error) {
  90. cctx, cancel := context.WithCancel(ctx)
  91. done := cancelWhenStop(cancel, l.stopCtx.Done())
  92. defer close(done)
  93. for {
  94. r := &pb.LeaseGrantRequest{TTL: ttl}
  95. resp, err := l.remote.LeaseGrant(cctx, r)
  96. if err == nil {
  97. gresp := &LeaseGrantResponse{
  98. ResponseHeader: resp.GetHeader(),
  99. ID: LeaseID(resp.ID),
  100. TTL: resp.TTL,
  101. Error: resp.Error,
  102. }
  103. return gresp, nil
  104. }
  105. if isHaltErr(cctx, err) {
  106. return nil, toErr(ctx, err)
  107. }
  108. if nerr := l.newStream(); nerr != nil {
  109. return nil, nerr
  110. }
  111. }
  112. }
  113. func (l *lessor) Revoke(ctx context.Context, id LeaseID) (*LeaseRevokeResponse, error) {
  114. cctx, cancel := context.WithCancel(ctx)
  115. done := cancelWhenStop(cancel, l.stopCtx.Done())
  116. defer close(done)
  117. for {
  118. r := &pb.LeaseRevokeRequest{ID: int64(id)}
  119. resp, err := l.remote.LeaseRevoke(cctx, r)
  120. if err == nil {
  121. return (*LeaseRevokeResponse)(resp), nil
  122. }
  123. if isHaltErr(ctx, err) {
  124. return nil, toErr(ctx, err)
  125. }
  126. if nerr := l.newStream(); nerr != nil {
  127. return nil, nerr
  128. }
  129. }
  130. }
  131. func (l *lessor) KeepAlive(ctx context.Context, id LeaseID) (<-chan *LeaseKeepAliveResponse, error) {
  132. ch := make(chan *LeaseKeepAliveResponse, leaseResponseChSize)
  133. l.mu.Lock()
  134. ka, ok := l.keepAlives[id]
  135. if !ok {
  136. // create fresh keep alive
  137. ka = &keepAlive{
  138. chs: []chan<- *LeaseKeepAliveResponse{ch},
  139. ctxs: []context.Context{ctx},
  140. deadline: time.Now(),
  141. donec: make(chan struct{}),
  142. }
  143. l.keepAlives[id] = ka
  144. } else {
  145. // add channel and context to existing keep alive
  146. ka.ctxs = append(ka.ctxs, ctx)
  147. ka.chs = append(ka.chs, ch)
  148. }
  149. l.mu.Unlock()
  150. go l.keepAliveCtxCloser(id, ctx, ka.donec)
  151. return ch, nil
  152. }
  153. func (l *lessor) KeepAliveOnce(ctx context.Context, id LeaseID) (*LeaseKeepAliveResponse, error) {
  154. cctx, cancel := context.WithCancel(ctx)
  155. done := cancelWhenStop(cancel, l.stopCtx.Done())
  156. defer close(done)
  157. for {
  158. resp, err := l.keepAliveOnce(cctx, id)
  159. if err == nil {
  160. if resp.TTL == 0 {
  161. err = rpctypes.ErrLeaseNotFound
  162. }
  163. return resp, err
  164. }
  165. if isHaltErr(ctx, err) {
  166. return nil, toErr(ctx, err)
  167. }
  168. if nerr := l.newStream(); nerr != nil {
  169. return nil, nerr
  170. }
  171. }
  172. }
  173. func (l *lessor) Close() error {
  174. l.stopCancel()
  175. <-l.donec
  176. return nil
  177. }
  178. func (l *lessor) keepAliveCtxCloser(id LeaseID, ctx context.Context, donec <-chan struct{}) {
  179. select {
  180. case <-donec:
  181. return
  182. case <-l.donec:
  183. return
  184. case <-ctx.Done():
  185. }
  186. l.mu.Lock()
  187. defer l.mu.Unlock()
  188. ka, ok := l.keepAlives[id]
  189. if !ok {
  190. return
  191. }
  192. // close channel and remove context if still associated with keep alive
  193. for i, c := range ka.ctxs {
  194. if c == ctx {
  195. close(ka.chs[i])
  196. ka.ctxs = append(ka.ctxs[:i], ka.ctxs[i+1:]...)
  197. ka.chs = append(ka.chs[:i], ka.chs[i+1:]...)
  198. break
  199. }
  200. }
  201. // remove if no one more listeners
  202. if len(ka.chs) == 0 {
  203. delete(l.keepAlives, id)
  204. }
  205. }
  206. func (l *lessor) keepAliveOnce(ctx context.Context, id LeaseID) (*LeaseKeepAliveResponse, error) {
  207. cctx, cancel := context.WithCancel(ctx)
  208. defer cancel()
  209. stream, err := l.remote.LeaseKeepAlive(cctx)
  210. if err != nil {
  211. return nil, toErr(ctx, err)
  212. }
  213. err = stream.Send(&pb.LeaseKeepAliveRequest{ID: int64(id)})
  214. if err != nil {
  215. return nil, toErr(ctx, err)
  216. }
  217. resp, rerr := stream.Recv()
  218. if rerr != nil {
  219. return nil, toErr(ctx, rerr)
  220. }
  221. karesp := &LeaseKeepAliveResponse{
  222. ResponseHeader: resp.GetHeader(),
  223. ID: LeaseID(resp.ID),
  224. TTL: resp.TTL,
  225. }
  226. return karesp, nil
  227. }
  228. func (l *lessor) recvKeepAliveLoop() {
  229. defer func() {
  230. l.mu.Lock()
  231. close(l.donec)
  232. for _, ka := range l.keepAlives {
  233. ka.Close()
  234. }
  235. l.keepAlives = make(map[LeaseID]*keepAlive)
  236. l.mu.Unlock()
  237. }()
  238. stream, serr := l.resetRecv()
  239. for serr == nil {
  240. resp, err := stream.Recv()
  241. if err != nil {
  242. if isHaltErr(l.stopCtx, err) {
  243. return
  244. }
  245. stream, serr = l.resetRecv()
  246. continue
  247. }
  248. l.recvKeepAlive(resp)
  249. }
  250. }
  251. // resetRecv opens a new lease stream and starts sending LeaseKeepAliveRequests
  252. func (l *lessor) resetRecv() (pb.Lease_LeaseKeepAliveClient, error) {
  253. if err := l.newStream(); err != nil {
  254. return nil, err
  255. }
  256. stream := l.getKeepAliveStream()
  257. go l.sendKeepAliveLoop(stream)
  258. return stream, nil
  259. }
  260. // recvKeepAlive updates a lease based on its LeaseKeepAliveResponse
  261. func (l *lessor) recvKeepAlive(resp *pb.LeaseKeepAliveResponse) {
  262. karesp := &LeaseKeepAliveResponse{
  263. ResponseHeader: resp.GetHeader(),
  264. ID: LeaseID(resp.ID),
  265. TTL: resp.TTL,
  266. }
  267. l.mu.Lock()
  268. defer l.mu.Unlock()
  269. ka, ok := l.keepAlives[karesp.ID]
  270. if !ok {
  271. return
  272. }
  273. if karesp.TTL <= 0 {
  274. // lease expired; close all keep alive channels
  275. delete(l.keepAlives, karesp.ID)
  276. ka.Close()
  277. return
  278. }
  279. // send update to all channels
  280. nextDeadline := time.Now().Add(1 + time.Duration(karesp.TTL/3)*time.Second)
  281. for _, ch := range ka.chs {
  282. select {
  283. case ch <- karesp:
  284. ka.deadline = nextDeadline
  285. default:
  286. }
  287. }
  288. }
  289. // sendKeepAliveLoop sends LeaseKeepAliveRequests for the lifetime of a lease stream
  290. func (l *lessor) sendKeepAliveLoop(stream pb.Lease_LeaseKeepAliveClient) {
  291. for {
  292. select {
  293. case <-time.After(500 * time.Millisecond):
  294. case <-stream.Context().Done():
  295. return
  296. case <-l.donec:
  297. return
  298. case <-l.stopCtx.Done():
  299. return
  300. }
  301. tosend := make([]LeaseID, 0)
  302. now := time.Now()
  303. l.mu.Lock()
  304. for id, ka := range l.keepAlives {
  305. if ka.deadline.Before(now) {
  306. tosend = append(tosend, id)
  307. }
  308. }
  309. l.mu.Unlock()
  310. for _, id := range tosend {
  311. r := &pb.LeaseKeepAliveRequest{ID: int64(id)}
  312. if err := stream.Send(r); err != nil {
  313. // TODO do something with this error?
  314. return
  315. }
  316. }
  317. }
  318. }
  319. func (l *lessor) getKeepAliveStream() pb.Lease_LeaseKeepAliveClient {
  320. l.mu.Lock()
  321. defer l.mu.Unlock()
  322. return l.stream
  323. }
  324. func (l *lessor) newStream() error {
  325. sctx, cancel := context.WithCancel(l.stopCtx)
  326. stream, err := l.remote.LeaseKeepAlive(sctx)
  327. if err != nil {
  328. cancel()
  329. return toErr(sctx, 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. }