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