watchable_store.go 13 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553
  1. // Copyright 2015 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 mvcc
  15. import (
  16. "sync"
  17. "time"
  18. "go.etcd.io/etcd/lease"
  19. "go.etcd.io/etcd/mvcc/backend"
  20. "go.etcd.io/etcd/mvcc/mvccpb"
  21. "go.etcd.io/etcd/pkg/traceutil"
  22. "go.uber.org/zap"
  23. )
  24. // non-const so modifiable by tests
  25. var (
  26. // chanBufLen is the length of the buffered chan
  27. // for sending out watched events.
  28. // TODO: find a good buf value. 1024 is just a random one that
  29. // seems to be reasonable.
  30. chanBufLen = 1024
  31. // maxWatchersPerSync is the number of watchers to sync in a single batch
  32. maxWatchersPerSync = 512
  33. )
  34. type watchable interface {
  35. watch(key, end []byte, startRev int64, id WatchID, ch chan<- WatchResponse, fcs ...FilterFunc) (*watcher, cancelFunc)
  36. progress(w *watcher)
  37. rev() int64
  38. }
  39. type watchableStore struct {
  40. *store
  41. // mu protects watcher groups and batches. It should never be locked
  42. // before locking store.mu to avoid deadlock.
  43. mu sync.RWMutex
  44. // victims are watcher batches that were blocked on the watch channel
  45. victims []watcherBatch
  46. victimc chan struct{}
  47. // contains all unsynced watchers that needs to sync with events that have happened
  48. unsynced watcherGroup
  49. // contains all synced watchers that are in sync with the progress of the store.
  50. // The key of the map is the key that the watcher watches on.
  51. synced watcherGroup
  52. stopc chan struct{}
  53. wg sync.WaitGroup
  54. }
  55. // cancelFunc updates unsynced and synced maps when running
  56. // cancel operations.
  57. type cancelFunc func()
  58. func New(lg *zap.Logger, b backend.Backend, le lease.Lessor, ig ConsistentIndexGetter, cfg StoreConfig) ConsistentWatchableKV {
  59. return newWatchableStore(lg, b, le, ig, cfg)
  60. }
  61. func newWatchableStore(lg *zap.Logger, b backend.Backend, le lease.Lessor, ig ConsistentIndexGetter, cfg StoreConfig) *watchableStore {
  62. s := &watchableStore{
  63. store: NewStore(lg, b, le, ig, cfg),
  64. victimc: make(chan struct{}, 1),
  65. unsynced: newWatcherGroup(),
  66. synced: newWatcherGroup(),
  67. stopc: make(chan struct{}),
  68. }
  69. s.store.ReadView = &readView{s}
  70. s.store.WriteView = &writeView{s}
  71. if s.le != nil {
  72. // use this store as the deleter so revokes trigger watch events
  73. s.le.SetRangeDeleter(func() lease.TxnDelete { return s.Write(traceutil.TODO()) })
  74. }
  75. s.wg.Add(2)
  76. go s.syncWatchersLoop()
  77. go s.syncVictimsLoop()
  78. return s
  79. }
  80. func (s *watchableStore) Close() error {
  81. close(s.stopc)
  82. s.wg.Wait()
  83. return s.store.Close()
  84. }
  85. func (s *watchableStore) NewWatchStream() WatchStream {
  86. watchStreamGauge.Inc()
  87. return &watchStream{
  88. watchable: s,
  89. ch: make(chan WatchResponse, chanBufLen),
  90. cancels: make(map[WatchID]cancelFunc),
  91. watchers: make(map[WatchID]*watcher),
  92. }
  93. }
  94. func (s *watchableStore) watch(key, end []byte, startRev int64, id WatchID, ch chan<- WatchResponse, fcs ...FilterFunc) (*watcher, cancelFunc) {
  95. wa := &watcher{
  96. key: key,
  97. end: end,
  98. minRev: startRev,
  99. id: id,
  100. ch: ch,
  101. fcs: fcs,
  102. }
  103. s.mu.Lock()
  104. s.revMu.RLock()
  105. synced := startRev > s.store.currentRev || startRev == 0
  106. if synced {
  107. wa.minRev = s.store.currentRev + 1
  108. if startRev > wa.minRev {
  109. wa.minRev = startRev
  110. }
  111. }
  112. if synced {
  113. s.synced.add(wa)
  114. } else {
  115. slowWatcherGauge.Inc()
  116. s.unsynced.add(wa)
  117. }
  118. s.revMu.RUnlock()
  119. s.mu.Unlock()
  120. watcherGauge.Inc()
  121. return wa, func() { s.cancelWatcher(wa) }
  122. }
  123. // cancelWatcher removes references of the watcher from the watchableStore
  124. func (s *watchableStore) cancelWatcher(wa *watcher) {
  125. for {
  126. s.mu.Lock()
  127. if s.unsynced.delete(wa) {
  128. slowWatcherGauge.Dec()
  129. break
  130. } else if s.synced.delete(wa) {
  131. break
  132. } else if wa.compacted {
  133. break
  134. } else if wa.ch == nil {
  135. // already canceled (e.g., cancel/close race)
  136. break
  137. }
  138. if !wa.victim {
  139. panic("watcher not victim but not in watch groups")
  140. }
  141. var victimBatch watcherBatch
  142. for _, wb := range s.victims {
  143. if wb[wa] != nil {
  144. victimBatch = wb
  145. break
  146. }
  147. }
  148. if victimBatch != nil {
  149. slowWatcherGauge.Dec()
  150. delete(victimBatch, wa)
  151. break
  152. }
  153. // victim being processed so not accessible; retry
  154. s.mu.Unlock()
  155. time.Sleep(time.Millisecond)
  156. }
  157. watcherGauge.Dec()
  158. wa.ch = nil
  159. s.mu.Unlock()
  160. }
  161. func (s *watchableStore) Restore(b backend.Backend) error {
  162. s.mu.Lock()
  163. defer s.mu.Unlock()
  164. err := s.store.Restore(b)
  165. if err != nil {
  166. return err
  167. }
  168. for wa := range s.synced.watchers {
  169. wa.restore = true
  170. s.unsynced.add(wa)
  171. }
  172. s.synced = newWatcherGroup()
  173. return nil
  174. }
  175. // syncWatchersLoop syncs the watcher in the unsynced map every 100ms.
  176. func (s *watchableStore) syncWatchersLoop() {
  177. defer s.wg.Done()
  178. for {
  179. s.mu.RLock()
  180. st := time.Now()
  181. lastUnsyncedWatchers := s.unsynced.size()
  182. s.mu.RUnlock()
  183. unsyncedWatchers := 0
  184. if lastUnsyncedWatchers > 0 {
  185. unsyncedWatchers = s.syncWatchers()
  186. }
  187. syncDuration := time.Since(st)
  188. waitDuration := 100 * time.Millisecond
  189. // more work pending?
  190. if unsyncedWatchers != 0 && lastUnsyncedWatchers > unsyncedWatchers {
  191. // be fair to other store operations by yielding time taken
  192. waitDuration = syncDuration
  193. }
  194. select {
  195. case <-time.After(waitDuration):
  196. case <-s.stopc:
  197. return
  198. }
  199. }
  200. }
  201. // syncVictimsLoop tries to write precomputed watcher responses to
  202. // watchers that had a blocked watcher channel
  203. func (s *watchableStore) syncVictimsLoop() {
  204. defer s.wg.Done()
  205. for {
  206. for s.moveVictims() != 0 {
  207. // try to update all victim watchers
  208. }
  209. s.mu.RLock()
  210. isEmpty := len(s.victims) == 0
  211. s.mu.RUnlock()
  212. var tickc <-chan time.Time
  213. if !isEmpty {
  214. tickc = time.After(10 * time.Millisecond)
  215. }
  216. select {
  217. case <-tickc:
  218. case <-s.victimc:
  219. case <-s.stopc:
  220. return
  221. }
  222. }
  223. }
  224. // moveVictims tries to update watches with already pending event data
  225. func (s *watchableStore) moveVictims() (moved int) {
  226. s.mu.Lock()
  227. victims := s.victims
  228. s.victims = nil
  229. s.mu.Unlock()
  230. var newVictim watcherBatch
  231. for _, wb := range victims {
  232. // try to send responses again
  233. for w, eb := range wb {
  234. // watcher has observed the store up to, but not including, w.minRev
  235. rev := w.minRev - 1
  236. if w.send(WatchResponse{WatchID: w.id, Events: eb.evs, Revision: rev}) {
  237. pendingEventsGauge.Add(float64(len(eb.evs)))
  238. } else {
  239. if newVictim == nil {
  240. newVictim = make(watcherBatch)
  241. }
  242. newVictim[w] = eb
  243. continue
  244. }
  245. moved++
  246. }
  247. // assign completed victim watchers to unsync/sync
  248. s.mu.Lock()
  249. s.store.revMu.RLock()
  250. curRev := s.store.currentRev
  251. for w, eb := range wb {
  252. if newVictim != nil && newVictim[w] != nil {
  253. // couldn't send watch response; stays victim
  254. continue
  255. }
  256. w.victim = false
  257. if eb.moreRev != 0 {
  258. w.minRev = eb.moreRev
  259. }
  260. if w.minRev <= curRev {
  261. s.unsynced.add(w)
  262. } else {
  263. slowWatcherGauge.Dec()
  264. s.synced.add(w)
  265. }
  266. }
  267. s.store.revMu.RUnlock()
  268. s.mu.Unlock()
  269. }
  270. if len(newVictim) > 0 {
  271. s.mu.Lock()
  272. s.victims = append(s.victims, newVictim)
  273. s.mu.Unlock()
  274. }
  275. return moved
  276. }
  277. // syncWatchers syncs unsynced watchers by:
  278. // 1. choose a set of watchers from the unsynced watcher group
  279. // 2. iterate over the set to get the minimum revision and remove compacted watchers
  280. // 3. use minimum revision to get all key-value pairs and send those events to watchers
  281. // 4. remove synced watchers in set from unsynced group and move to synced group
  282. func (s *watchableStore) syncWatchers() int {
  283. s.mu.Lock()
  284. defer s.mu.Unlock()
  285. if s.unsynced.size() == 0 {
  286. return 0
  287. }
  288. s.store.revMu.RLock()
  289. defer s.store.revMu.RUnlock()
  290. // in order to find key-value pairs from unsynced watchers, we need to
  291. // find min revision index, and these revisions can be used to
  292. // query the backend store of key-value pairs
  293. curRev := s.store.currentRev
  294. compactionRev := s.store.compactMainRev
  295. wg, minRev := s.unsynced.choose(maxWatchersPerSync, curRev, compactionRev)
  296. minBytes, maxBytes := newRevBytes(), newRevBytes()
  297. revToBytes(revision{main: minRev}, minBytes)
  298. revToBytes(revision{main: curRev + 1}, maxBytes)
  299. // UnsafeRange returns keys and values. And in boltdb, keys are revisions.
  300. // values are actual key-value pairs in backend.
  301. tx := s.store.b.ReadTx()
  302. tx.RLock()
  303. revs, vs := tx.UnsafeRange(keyBucketName, minBytes, maxBytes, 0)
  304. var evs []mvccpb.Event
  305. if s.store != nil && s.store.lg != nil {
  306. evs = kvsToEvents(s.store.lg, wg, revs, vs)
  307. } else {
  308. // TODO: remove this in v3.5
  309. evs = kvsToEvents(nil, wg, revs, vs)
  310. }
  311. tx.RUnlock()
  312. var victims watcherBatch
  313. wb := newWatcherBatch(wg, evs)
  314. for w := range wg.watchers {
  315. w.minRev = curRev + 1
  316. eb, ok := wb[w]
  317. if !ok {
  318. // bring un-notified watcher to synced
  319. s.synced.add(w)
  320. s.unsynced.delete(w)
  321. continue
  322. }
  323. if eb.moreRev != 0 {
  324. w.minRev = eb.moreRev
  325. }
  326. if w.send(WatchResponse{WatchID: w.id, Events: eb.evs, Revision: curRev}) {
  327. pendingEventsGauge.Add(float64(len(eb.evs)))
  328. } else {
  329. if victims == nil {
  330. victims = make(watcherBatch)
  331. }
  332. w.victim = true
  333. }
  334. if w.victim {
  335. victims[w] = eb
  336. } else {
  337. if eb.moreRev != 0 {
  338. // stay unsynced; more to read
  339. continue
  340. }
  341. s.synced.add(w)
  342. }
  343. s.unsynced.delete(w)
  344. }
  345. s.addVictim(victims)
  346. vsz := 0
  347. for _, v := range s.victims {
  348. vsz += len(v)
  349. }
  350. slowWatcherGauge.Set(float64(s.unsynced.size() + vsz))
  351. return s.unsynced.size()
  352. }
  353. // kvsToEvents gets all events for the watchers from all key-value pairs
  354. func kvsToEvents(lg *zap.Logger, wg *watcherGroup, revs, vals [][]byte) (evs []mvccpb.Event) {
  355. for i, v := range vals {
  356. var kv mvccpb.KeyValue
  357. if err := kv.Unmarshal(v); err != nil {
  358. if lg != nil {
  359. lg.Panic("failed to unmarshal mvccpb.KeyValue", zap.Error(err))
  360. } else {
  361. plog.Panicf("cannot unmarshal event: %v", err)
  362. }
  363. }
  364. if !wg.contains(string(kv.Key)) {
  365. continue
  366. }
  367. ty := mvccpb.PUT
  368. if isTombstone(revs[i]) {
  369. ty = mvccpb.DELETE
  370. // patch in mod revision so watchers won't skip
  371. kv.ModRevision = bytesToRev(revs[i]).main
  372. }
  373. evs = append(evs, mvccpb.Event{Kv: &kv, Type: ty})
  374. }
  375. return evs
  376. }
  377. // notify notifies the fact that given event at the given rev just happened to
  378. // watchers that watch on the key of the event.
  379. func (s *watchableStore) notify(rev int64, evs []mvccpb.Event) {
  380. var victim watcherBatch
  381. for w, eb := range newWatcherBatch(&s.synced, evs) {
  382. if eb.revs != 1 {
  383. if s.store != nil && s.store.lg != nil {
  384. s.store.lg.Panic(
  385. "unexpected multiple revisions in watch notification",
  386. zap.Int("number-of-revisions", eb.revs),
  387. )
  388. } else {
  389. plog.Panicf("unexpected multiple revisions in notification")
  390. }
  391. }
  392. if w.send(WatchResponse{WatchID: w.id, Events: eb.evs, Revision: rev}) {
  393. pendingEventsGauge.Add(float64(len(eb.evs)))
  394. } else {
  395. // move slow watcher to victims
  396. w.minRev = rev + 1
  397. if victim == nil {
  398. victim = make(watcherBatch)
  399. }
  400. w.victim = true
  401. victim[w] = eb
  402. s.synced.delete(w)
  403. slowWatcherGauge.Inc()
  404. }
  405. }
  406. s.addVictim(victim)
  407. }
  408. func (s *watchableStore) addVictim(victim watcherBatch) {
  409. if victim == nil {
  410. return
  411. }
  412. s.victims = append(s.victims, victim)
  413. select {
  414. case s.victimc <- struct{}{}:
  415. default:
  416. }
  417. }
  418. func (s *watchableStore) rev() int64 { return s.store.Rev() }
  419. func (s *watchableStore) progress(w *watcher) {
  420. s.mu.RLock()
  421. defer s.mu.RUnlock()
  422. if _, ok := s.synced.watchers[w]; ok {
  423. w.send(WatchResponse{WatchID: w.id, Revision: s.rev()})
  424. // If the ch is full, this watcher is receiving events.
  425. // We do not need to send progress at all.
  426. }
  427. }
  428. type watcher struct {
  429. // the watcher key
  430. key []byte
  431. // end indicates the end of the range to watch.
  432. // If end is set, the watcher is on a range.
  433. end []byte
  434. // victim is set when ch is blocked and undergoing victim processing
  435. victim bool
  436. // compacted is set when the watcher is removed because of compaction
  437. compacted bool
  438. // restore is true when the watcher is being restored from leader snapshot
  439. // which means that this watcher has just been moved from "synced" to "unsynced"
  440. // watcher group, possibly with a future revision when it was first added
  441. // to the synced watcher
  442. // "unsynced" watcher revision must always be <= current revision,
  443. // except when the watcher were to be moved from "synced" watcher group
  444. restore bool
  445. // minRev is the minimum revision update the watcher will accept
  446. minRev int64
  447. id WatchID
  448. fcs []FilterFunc
  449. // a chan to send out the watch response.
  450. // The chan might be shared with other watchers.
  451. ch chan<- WatchResponse
  452. }
  453. func (w *watcher) send(wr WatchResponse) bool {
  454. progressEvent := len(wr.Events) == 0
  455. if len(w.fcs) != 0 {
  456. ne := make([]mvccpb.Event, 0, len(wr.Events))
  457. for i := range wr.Events {
  458. filtered := false
  459. for _, filter := range w.fcs {
  460. if filter(wr.Events[i]) {
  461. filtered = true
  462. break
  463. }
  464. }
  465. if !filtered {
  466. ne = append(ne, wr.Events[i])
  467. }
  468. }
  469. wr.Events = ne
  470. }
  471. // if all events are filtered out, we should send nothing.
  472. if !progressEvent && len(wr.Events) == 0 {
  473. return true
  474. }
  475. select {
  476. case w.ch <- wr:
  477. return true
  478. default:
  479. return false
  480. }
  481. }