lease.go 8.1 KB

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