lease.go 8.3 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377
  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. // Lease keeps internal routines and connections for efficient communication with etcd server.
  43. // After using Lease, call Close() to release all related resources.
  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]chan *LeaseKeepAliveResponse
  58. deadlines map[lease.LeaseID]time.Time
  59. }
  60. func NewLease(c *Client) Lease {
  61. l := &lessor{
  62. c: c,
  63. conn: c.ActiveConnection(),
  64. donec: make(chan struct{}),
  65. keepAlives: make(map[lease.LeaseID]chan *LeaseKeepAliveResponse),
  66. deadlines: make(map[lease.LeaseID]time.Time),
  67. }
  68. l.remote = pb.NewLeaseClient(l.conn)
  69. l.stopCtx, l.stopCancel = context.WithCancel(context.Background())
  70. go l.recvKeepAliveLoop()
  71. return l
  72. }
  73. func (l *lessor) Create(ctx context.Context, ttl int64) (*LeaseCreateResponse, error) {
  74. cctx, cancel := context.WithCancel(ctx)
  75. done := cancelWhenStop(cancel, l.stopCtx.Done())
  76. defer close(done)
  77. for {
  78. r := &pb.LeaseCreateRequest{TTL: ttl}
  79. resp, err := l.getRemote().LeaseCreate(cctx, r)
  80. if err == nil {
  81. return (*LeaseCreateResponse)(resp), nil
  82. }
  83. if isRPCError(err) {
  84. return nil, err
  85. }
  86. if nerr := l.switchRemoteAndStream(err); nerr != nil {
  87. return nil, nerr
  88. }
  89. }
  90. }
  91. func (l *lessor) Revoke(ctx context.Context, id lease.LeaseID) (*LeaseRevokeResponse, error) {
  92. cctx, cancel := context.WithCancel(ctx)
  93. done := cancelWhenStop(cancel, l.stopCtx.Done())
  94. defer close(done)
  95. for {
  96. r := &pb.LeaseRevokeRequest{ID: int64(id)}
  97. resp, err := l.getRemote().LeaseRevoke(cctx, r)
  98. if err == nil {
  99. return (*LeaseRevokeResponse)(resp), nil
  100. }
  101. if isRPCError(err) {
  102. return nil, err
  103. }
  104. if nerr := l.switchRemoteAndStream(err); nerr != nil {
  105. return nil, nerr
  106. }
  107. }
  108. }
  109. func (l *lessor) KeepAlive(ctx context.Context, id lease.LeaseID) (<-chan *LeaseKeepAliveResponse, error) {
  110. lc := make(chan *LeaseKeepAliveResponse, leaseResponseChSize)
  111. // todo: add concellation based on the passed in ctx
  112. l.mu.Lock()
  113. _, ok := l.keepAlives[id]
  114. if !ok {
  115. l.keepAlives[id] = lc
  116. l.deadlines[id] = time.Now()
  117. l.mu.Unlock()
  118. return lc, nil
  119. }
  120. l.mu.Unlock()
  121. resp, err := l.KeepAliveOnce(ctx, id)
  122. if err != nil {
  123. return nil, err
  124. }
  125. lc <- resp
  126. return lc, nil
  127. }
  128. func (l *lessor) KeepAliveOnce(ctx context.Context, id lease.LeaseID) (*LeaseKeepAliveResponse, error) {
  129. cctx, cancel := context.WithCancel(ctx)
  130. done := cancelWhenStop(cancel, l.stopCtx.Done())
  131. defer close(done)
  132. for {
  133. resp, err := l.keepAliveOnce(cctx, id)
  134. if err == nil {
  135. return resp, err
  136. }
  137. nerr := l.switchRemoteAndStream(err)
  138. if nerr != nil {
  139. return nil, nerr
  140. }
  141. }
  142. }
  143. func (l *lessor) Close() error {
  144. l.stopCancel()
  145. <-l.donec
  146. return nil
  147. }
  148. func (l *lessor) keepAliveOnce(ctx context.Context, id lease.LeaseID) (*LeaseKeepAliveResponse, error) {
  149. stream, err := l.getRemote().LeaseKeepAlive(ctx)
  150. if err != nil {
  151. return nil, err
  152. }
  153. err = stream.Send(&pb.LeaseKeepAliveRequest{ID: int64(id)})
  154. if err != nil {
  155. return nil, err
  156. }
  157. resp, rerr := stream.Recv()
  158. if rerr != nil {
  159. return nil, rerr
  160. }
  161. return (*LeaseKeepAliveResponse)(resp), nil
  162. }
  163. func (l *lessor) recvKeepAliveLoop() {
  164. defer func() {
  165. l.stopCancel()
  166. close(l.donec)
  167. for _, ch := range l.keepAlives {
  168. close(ch)
  169. }
  170. }()
  171. stream, serr := l.resetRecv()
  172. for {
  173. resp, err := stream.Recv()
  174. if err != nil {
  175. if isRPCError(err) {
  176. return
  177. }
  178. if stream, serr = l.resetRecv(); serr != nil {
  179. return
  180. }
  181. continue
  182. }
  183. l.recvKeepAlive(resp)
  184. }
  185. }
  186. // resetRecv opens a new lease stream and starts sending LeaseKeepAliveRequests
  187. func (l *lessor) resetRecv() (pb.Lease_LeaseKeepAliveClient, error) {
  188. if err := l.switchRemoteAndStream(nil); err != nil {
  189. return nil, err
  190. }
  191. stream := l.getKeepAliveStream()
  192. go l.sendKeepAliveLoop(stream)
  193. return stream, nil
  194. }
  195. // recvKeepAlive updates a lease based on its LeaseKeepAliveResponse
  196. func (l *lessor) recvKeepAlive(resp *pb.LeaseKeepAliveResponse) {
  197. l.mu.Lock()
  198. defer l.mu.Unlock()
  199. lch, ok := l.keepAlives[lease.LeaseID(resp.ID)]
  200. if !ok {
  201. return
  202. }
  203. if resp.TTL <= 0 {
  204. close(lch)
  205. delete(l.deadlines, lease.LeaseID(resp.ID))
  206. delete(l.keepAlives, lease.LeaseID(resp.ID))
  207. return
  208. }
  209. select {
  210. case lch <- (*LeaseKeepAliveResponse)(resp):
  211. l.deadlines[lease.LeaseID(resp.ID)] =
  212. time.Now().Add(1 + time.Duration(resp.TTL/3)*time.Second)
  213. default:
  214. }
  215. }
  216. // sendKeepAliveLoop sends LeaseKeepAliveRequests for the lifetime of a lease stream
  217. func (l *lessor) sendKeepAliveLoop(stream pb.Lease_LeaseKeepAliveClient) {
  218. for {
  219. select {
  220. case <-time.After(500 * time.Millisecond):
  221. case <-l.donec:
  222. return
  223. case <-l.stopCtx.Done():
  224. return
  225. }
  226. tosend := make([]lease.LeaseID, 0)
  227. now := time.Now()
  228. l.mu.Lock()
  229. for id, d := range l.deadlines {
  230. if d.Before(now) {
  231. tosend = append(tosend, id)
  232. }
  233. }
  234. l.mu.Unlock()
  235. for _, id := range tosend {
  236. r := &pb.LeaseKeepAliveRequest{ID: int64(id)}
  237. if err := stream.Send(r); err != nil {
  238. // TODO do something with this error?
  239. return
  240. }
  241. }
  242. }
  243. }
  244. func (l *lessor) getRemote() pb.LeaseClient {
  245. l.mu.Lock()
  246. defer l.mu.Unlock()
  247. return l.remote
  248. }
  249. func (l *lessor) getKeepAliveStream() pb.Lease_LeaseKeepAliveClient {
  250. l.mu.Lock()
  251. defer l.mu.Unlock()
  252. return l.stream
  253. }
  254. func (l *lessor) switchRemoteAndStream(prevErr error) error {
  255. l.mu.Lock()
  256. conn := l.conn
  257. l.mu.Unlock()
  258. var (
  259. err error
  260. newConn *grpc.ClientConn
  261. )
  262. if prevErr != nil {
  263. conn.Close()
  264. newConn, err = l.c.retryConnection(conn, prevErr)
  265. if err != nil {
  266. return err
  267. }
  268. }
  269. l.mu.Lock()
  270. if newConn != nil {
  271. l.conn = newConn
  272. }
  273. l.remote = pb.NewLeaseClient(l.conn)
  274. l.mu.Unlock()
  275. serr := l.newStream()
  276. if serr != nil {
  277. return serr
  278. }
  279. return nil
  280. }
  281. func (l *lessor) newStream() error {
  282. sctx, cancel := context.WithCancel(l.stopCtx)
  283. stream, err := l.getRemote().LeaseKeepAlive(sctx)
  284. if err != nil {
  285. cancel()
  286. return err
  287. }
  288. l.mu.Lock()
  289. defer l.mu.Unlock()
  290. if l.stream != nil && l.streamCancel != nil {
  291. l.stream.CloseSend()
  292. l.streamCancel()
  293. }
  294. l.streamCancel = cancel
  295. l.stream = stream
  296. return nil
  297. }
  298. // cancelWhenStop calls cancel when the given stopc fires. It returns a done chan. done
  299. // should be closed when the work is finished. When done fires, cancelWhenStop will release
  300. // its internal resource.
  301. func cancelWhenStop(cancel context.CancelFunc, stopc <-chan struct{}) chan<- struct{} {
  302. done := make(chan struct{}, 1)
  303. go func() {
  304. select {
  305. case <-stopc:
  306. case <-done:
  307. }
  308. cancel()
  309. }()
  310. return done
  311. }