watch_test.go 24 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844845846847848849850851852853854855856857858859860861862863864865866867868869870871872873874875876877878879880881882883884885886887888889890891892893
  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 integration
  15. import (
  16. "fmt"
  17. "math/rand"
  18. "reflect"
  19. "sort"
  20. "testing"
  21. "time"
  22. "github.com/coreos/etcd/clientv3"
  23. "github.com/coreos/etcd/etcdserver/api/v3rpc"
  24. "github.com/coreos/etcd/etcdserver/api/v3rpc/rpctypes"
  25. "github.com/coreos/etcd/integration"
  26. mvccpb "github.com/coreos/etcd/mvcc/mvccpb"
  27. "github.com/coreos/etcd/pkg/testutil"
  28. "golang.org/x/net/context"
  29. "google.golang.org/grpc"
  30. )
  31. type watcherTest func(*testing.T, *watchctx)
  32. type watchctx struct {
  33. clus *integration.ClusterV3
  34. w clientv3.Watcher
  35. wclient *clientv3.Client
  36. kv clientv3.KV
  37. wclientMember int
  38. kvMember int
  39. ch clientv3.WatchChan
  40. }
  41. func runWatchTest(t *testing.T, f watcherTest) {
  42. defer testutil.AfterTest(t)
  43. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 3})
  44. defer clus.Terminate(t)
  45. wclientMember := rand.Intn(3)
  46. wclient := clus.Client(wclientMember)
  47. w := clientv3.NewWatcher(wclient)
  48. defer w.Close()
  49. // select a different client from wclient so puts succeed if
  50. // a test knocks out the watcher client
  51. kvMember := rand.Intn(3)
  52. for kvMember == wclientMember {
  53. kvMember = rand.Intn(3)
  54. }
  55. kvclient := clus.Client(kvMember)
  56. kv := clientv3.NewKV(kvclient)
  57. wctx := &watchctx{clus, w, wclient, kv, wclientMember, kvMember, nil}
  58. f(t, wctx)
  59. }
  60. // TestWatchMultiWatcher modifies multiple keys and observes the changes.
  61. func TestWatchMultiWatcher(t *testing.T) {
  62. runWatchTest(t, testWatchMultiWatcher)
  63. }
  64. func testWatchMultiWatcher(t *testing.T, wctx *watchctx) {
  65. numKeyUpdates := 4
  66. keys := []string{"foo", "bar", "baz"}
  67. donec := make(chan struct{})
  68. readyc := make(chan struct{})
  69. for _, k := range keys {
  70. // key watcher
  71. go func(key string) {
  72. ch := wctx.w.Watch(context.TODO(), key)
  73. if ch == nil {
  74. t.Fatalf("expected watcher channel, got nil")
  75. }
  76. readyc <- struct{}{}
  77. for i := 0; i < numKeyUpdates; i++ {
  78. resp, ok := <-ch
  79. if !ok {
  80. t.Fatalf("watcher unexpectedly closed")
  81. }
  82. v := fmt.Sprintf("%s-%d", key, i)
  83. gotv := string(resp.Events[0].Kv.Value)
  84. if gotv != v {
  85. t.Errorf("#%d: got %s, wanted %s", i, gotv, v)
  86. }
  87. }
  88. donec <- struct{}{}
  89. }(k)
  90. }
  91. // prefix watcher on "b" (bar and baz)
  92. go func() {
  93. prefixc := wctx.w.Watch(context.TODO(), "b", clientv3.WithPrefix())
  94. if prefixc == nil {
  95. t.Fatalf("expected watcher channel, got nil")
  96. }
  97. readyc <- struct{}{}
  98. evs := []*clientv3.Event{}
  99. for i := 0; i < numKeyUpdates*2; i++ {
  100. resp, ok := <-prefixc
  101. if !ok {
  102. t.Fatalf("watcher unexpectedly closed")
  103. }
  104. evs = append(evs, resp.Events...)
  105. }
  106. // check response
  107. expected := []string{}
  108. bkeys := []string{"bar", "baz"}
  109. for _, k := range bkeys {
  110. for i := 0; i < numKeyUpdates; i++ {
  111. expected = append(expected, fmt.Sprintf("%s-%d", k, i))
  112. }
  113. }
  114. got := []string{}
  115. for _, ev := range evs {
  116. got = append(got, string(ev.Kv.Value))
  117. }
  118. sort.Strings(got)
  119. if !reflect.DeepEqual(expected, got) {
  120. t.Errorf("got %v, expected %v", got, expected)
  121. }
  122. // ensure no extra data
  123. select {
  124. case resp, ok := <-prefixc:
  125. if !ok {
  126. t.Fatalf("watcher unexpectedly closed")
  127. }
  128. t.Fatalf("unexpected event %+v", resp)
  129. case <-time.After(time.Second):
  130. }
  131. donec <- struct{}{}
  132. }()
  133. // wait for watcher bring up
  134. for i := 0; i < len(keys)+1; i++ {
  135. <-readyc
  136. }
  137. // generate events
  138. ctx := context.TODO()
  139. for i := 0; i < numKeyUpdates; i++ {
  140. for _, k := range keys {
  141. v := fmt.Sprintf("%s-%d", k, i)
  142. if _, err := wctx.kv.Put(ctx, k, v); err != nil {
  143. t.Fatal(err)
  144. }
  145. }
  146. }
  147. // wait for watcher shutdown
  148. for i := 0; i < len(keys)+1; i++ {
  149. <-donec
  150. }
  151. }
  152. // TestWatchRange tests watcher creates ranges
  153. func TestWatchRange(t *testing.T) {
  154. runWatchTest(t, testWatchRange)
  155. }
  156. func testWatchRange(t *testing.T, wctx *watchctx) {
  157. if wctx.ch = wctx.w.Watch(context.TODO(), "a", clientv3.WithRange("c")); wctx.ch == nil {
  158. t.Fatalf("expected non-nil channel")
  159. }
  160. putAndWatch(t, wctx, "a", "a")
  161. putAndWatch(t, wctx, "b", "b")
  162. putAndWatch(t, wctx, "bar", "bar")
  163. }
  164. // TestWatchReconnRequest tests the send failure path when requesting a watcher.
  165. func TestWatchReconnRequest(t *testing.T) {
  166. runWatchTest(t, testWatchReconnRequest)
  167. }
  168. func testWatchReconnRequest(t *testing.T, wctx *watchctx) {
  169. donec, stopc := make(chan struct{}), make(chan struct{}, 1)
  170. go func() {
  171. timer := time.After(2 * time.Second)
  172. defer close(donec)
  173. // take down watcher connection
  174. for {
  175. wctx.clus.Members[wctx.wclientMember].DropConnections()
  176. select {
  177. case <-timer:
  178. // spinning on close may live lock reconnection
  179. return
  180. case <-stopc:
  181. return
  182. default:
  183. }
  184. }
  185. }()
  186. // should reconnect when requesting watch
  187. if wctx.ch = wctx.w.Watch(context.TODO(), "a"); wctx.ch == nil {
  188. t.Fatalf("expected non-nil channel")
  189. }
  190. // wait for disconnections to stop
  191. stopc <- struct{}{}
  192. <-donec
  193. // spinning on dropping connections may trigger a leader election
  194. // due to resource starvation; l-read to ensure the cluster is stable
  195. ctx, cancel := context.WithTimeout(context.TODO(), 30*time.Second)
  196. if _, err := wctx.kv.Get(ctx, "_"); err != nil {
  197. t.Fatal(err)
  198. }
  199. cancel()
  200. // ensure watcher works
  201. putAndWatch(t, wctx, "a", "a")
  202. }
  203. // TestWatchReconnInit tests watcher resumes correctly if connection lost
  204. // before any data was sent.
  205. func TestWatchReconnInit(t *testing.T) {
  206. runWatchTest(t, testWatchReconnInit)
  207. }
  208. func testWatchReconnInit(t *testing.T, wctx *watchctx) {
  209. if wctx.ch = wctx.w.Watch(context.TODO(), "a"); wctx.ch == nil {
  210. t.Fatalf("expected non-nil channel")
  211. }
  212. wctx.clus.Members[wctx.wclientMember].DropConnections()
  213. // watcher should recover
  214. putAndWatch(t, wctx, "a", "a")
  215. }
  216. // TestWatchReconnRunning tests watcher resumes correctly if connection lost
  217. // after data was sent.
  218. func TestWatchReconnRunning(t *testing.T) {
  219. runWatchTest(t, testWatchReconnRunning)
  220. }
  221. func testWatchReconnRunning(t *testing.T, wctx *watchctx) {
  222. if wctx.ch = wctx.w.Watch(context.TODO(), "a"); wctx.ch == nil {
  223. t.Fatalf("expected non-nil channel")
  224. }
  225. putAndWatch(t, wctx, "a", "a")
  226. // take down watcher connection
  227. wctx.clus.Members[wctx.wclientMember].DropConnections()
  228. // watcher should recover
  229. putAndWatch(t, wctx, "a", "b")
  230. }
  231. // TestWatchCancelImmediate ensures a closed channel is returned
  232. // if the context is cancelled.
  233. func TestWatchCancelImmediate(t *testing.T) {
  234. runWatchTest(t, testWatchCancelImmediate)
  235. }
  236. func testWatchCancelImmediate(t *testing.T, wctx *watchctx) {
  237. ctx, cancel := context.WithCancel(context.Background())
  238. cancel()
  239. wch := wctx.w.Watch(ctx, "a")
  240. select {
  241. case wresp, ok := <-wch:
  242. if ok {
  243. t.Fatalf("read wch got %v; expected closed channel", wresp)
  244. }
  245. default:
  246. t.Fatalf("closed watcher channel should not block")
  247. }
  248. }
  249. // TestWatchCancelInit tests watcher closes correctly after no events.
  250. func TestWatchCancelInit(t *testing.T) {
  251. runWatchTest(t, testWatchCancelInit)
  252. }
  253. func testWatchCancelInit(t *testing.T, wctx *watchctx) {
  254. ctx, cancel := context.WithCancel(context.Background())
  255. if wctx.ch = wctx.w.Watch(ctx, "a"); wctx.ch == nil {
  256. t.Fatalf("expected non-nil watcher channel")
  257. }
  258. cancel()
  259. select {
  260. case <-time.After(time.Second):
  261. t.Fatalf("took too long to cancel")
  262. case _, ok := <-wctx.ch:
  263. if ok {
  264. t.Fatalf("expected watcher channel to close")
  265. }
  266. }
  267. }
  268. // TestWatchCancelRunning tests watcher closes correctly after events.
  269. func TestWatchCancelRunning(t *testing.T) {
  270. runWatchTest(t, testWatchCancelRunning)
  271. }
  272. func testWatchCancelRunning(t *testing.T, wctx *watchctx) {
  273. ctx, cancel := context.WithCancel(context.Background())
  274. if wctx.ch = wctx.w.Watch(ctx, "a"); wctx.ch == nil {
  275. t.Fatalf("expected non-nil watcher channel")
  276. }
  277. if _, err := wctx.kv.Put(ctx, "a", "a"); err != nil {
  278. t.Fatal(err)
  279. }
  280. cancel()
  281. select {
  282. case <-time.After(time.Second):
  283. t.Fatalf("took too long to cancel")
  284. case v, ok := <-wctx.ch:
  285. if !ok {
  286. // closed before getting put; OK
  287. break
  288. }
  289. // got the PUT; should close next
  290. select {
  291. case <-time.After(time.Second):
  292. t.Fatalf("took too long to close")
  293. case v, ok = <-wctx.ch:
  294. if ok {
  295. t.Fatalf("expected watcher channel to close, got %v", v)
  296. }
  297. }
  298. }
  299. }
  300. func putAndWatch(t *testing.T, wctx *watchctx, key, val string) {
  301. if _, err := wctx.kv.Put(context.TODO(), key, val); err != nil {
  302. t.Fatal(err)
  303. }
  304. select {
  305. case <-time.After(5 * time.Second):
  306. t.Fatalf("watch timed out")
  307. case v, ok := <-wctx.ch:
  308. if !ok {
  309. t.Fatalf("unexpected watch close")
  310. }
  311. if string(v.Events[0].Kv.Value) != val {
  312. t.Fatalf("bad value got %v, wanted %v", v.Events[0].Kv.Value, val)
  313. }
  314. }
  315. }
  316. // TestWatchResumeComapcted checks that the watcher gracefully closes in case
  317. // that it tries to resume to a revision that's been compacted out of the store.
  318. func TestWatchResumeCompacted(t *testing.T) {
  319. defer testutil.AfterTest(t)
  320. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 3})
  321. defer clus.Terminate(t)
  322. // create a waiting watcher at rev 1
  323. w := clientv3.NewWatcher(clus.Client(0))
  324. defer w.Close()
  325. wch := w.Watch(context.Background(), "foo", clientv3.WithRev(1))
  326. select {
  327. case w := <-wch:
  328. t.Errorf("unexpected message from wch %v", w)
  329. default:
  330. }
  331. clus.Members[0].Stop(t)
  332. ticker := time.After(time.Second * 10)
  333. for clus.WaitLeader(t) <= 0 {
  334. select {
  335. case <-ticker:
  336. t.Fatalf("failed to wait for new leader")
  337. default:
  338. time.Sleep(10 * time.Millisecond)
  339. }
  340. }
  341. // put some data and compact away
  342. kv := clientv3.NewKV(clus.Client(1))
  343. for i := 0; i < 5; i++ {
  344. if _, err := kv.Put(context.TODO(), "foo", "bar"); err != nil {
  345. t.Fatal(err)
  346. }
  347. }
  348. if _, err := kv.Compact(context.TODO(), 3); err != nil {
  349. t.Fatal(err)
  350. }
  351. clus.Members[0].Restart(t)
  352. // get compacted error message
  353. wresp, ok := <-wch
  354. if !ok {
  355. t.Fatalf("expected wresp, but got closed channel")
  356. }
  357. if wresp.Err() != rpctypes.ErrCompacted {
  358. t.Fatalf("wresp.Err() expected %v, but got %v", rpctypes.ErrCompacted, wresp.Err())
  359. }
  360. // ensure the channel is closed
  361. if wresp, ok = <-wch; ok {
  362. t.Fatalf("expected closed channel, but got %v", wresp)
  363. }
  364. }
  365. // TestWatchCompactRevision ensures the CompactRevision error is given on a
  366. // compaction event ahead of a watcher.
  367. func TestWatchCompactRevision(t *testing.T) {
  368. defer testutil.AfterTest(t)
  369. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  370. defer clus.Terminate(t)
  371. // set some keys
  372. kv := clientv3.NewKV(clus.RandClient())
  373. for i := 0; i < 5; i++ {
  374. if _, err := kv.Put(context.TODO(), "foo", "bar"); err != nil {
  375. t.Fatal(err)
  376. }
  377. }
  378. w := clientv3.NewWatcher(clus.RandClient())
  379. defer w.Close()
  380. if _, err := kv.Compact(context.TODO(), 4); err != nil {
  381. t.Fatal(err)
  382. }
  383. wch := w.Watch(context.Background(), "foo", clientv3.WithRev(2))
  384. // get compacted error message
  385. wresp, ok := <-wch
  386. if !ok {
  387. t.Fatalf("expected wresp, but got closed channel")
  388. }
  389. if wresp.Err() != rpctypes.ErrCompacted {
  390. t.Fatalf("wresp.Err() expected %v, but got %v", rpctypes.ErrCompacted, wresp.Err())
  391. }
  392. // ensure the channel is closed
  393. if wresp, ok = <-wch; ok {
  394. t.Fatalf("expected closed channel, but got %v", wresp)
  395. }
  396. }
  397. func TestWatchWithProgressNotify(t *testing.T) { testWatchWithProgressNotify(t, true) }
  398. func TestWatchWithProgressNotifyNoEvent(t *testing.T) { testWatchWithProgressNotify(t, false) }
  399. func testWatchWithProgressNotify(t *testing.T, watchOnPut bool) {
  400. defer testutil.AfterTest(t)
  401. // accelerate report interval so test terminates quickly
  402. oldpi := v3rpc.GetProgressReportInterval()
  403. // using atomics to avoid race warnings
  404. v3rpc.SetProgressReportInterval(3 * time.Second)
  405. pi := 3 * time.Second
  406. defer func() { v3rpc.SetProgressReportInterval(oldpi) }()
  407. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 3})
  408. defer clus.Terminate(t)
  409. wc := clientv3.NewWatcher(clus.RandClient())
  410. defer wc.Close()
  411. opts := []clientv3.OpOption{clientv3.WithProgressNotify()}
  412. if watchOnPut {
  413. opts = append(opts, clientv3.WithPrefix())
  414. }
  415. rch := wc.Watch(context.Background(), "foo", opts...)
  416. select {
  417. case resp := <-rch: // wait for notification
  418. if len(resp.Events) != 0 {
  419. t.Fatalf("resp.Events expected none, got %+v", resp.Events)
  420. }
  421. case <-time.After(2 * pi):
  422. t.Fatalf("watch response expected in %v, but timed out", pi)
  423. }
  424. kvc := clientv3.NewKV(clus.RandClient())
  425. if _, err := kvc.Put(context.TODO(), "foox", "bar"); err != nil {
  426. t.Fatal(err)
  427. }
  428. select {
  429. case resp := <-rch:
  430. if resp.Header.Revision != 2 {
  431. t.Fatalf("resp.Header.Revision expected 2, got %d", resp.Header.Revision)
  432. }
  433. if watchOnPut { // wait for put if watch on the put key
  434. ev := []*clientv3.Event{{Type: clientv3.EventTypePut,
  435. Kv: &mvccpb.KeyValue{Key: []byte("foox"), Value: []byte("bar"), CreateRevision: 2, ModRevision: 2, Version: 1}}}
  436. if !reflect.DeepEqual(ev, resp.Events) {
  437. t.Fatalf("expected %+v, got %+v", ev, resp.Events)
  438. }
  439. } else if len(resp.Events) != 0 { // wait for notification otherwise
  440. t.Fatalf("expected no events, but got %+v", resp.Events)
  441. }
  442. case <-time.After(time.Duration(1.5 * float64(pi))):
  443. t.Fatalf("watch response expected in %v, but timed out", pi)
  444. }
  445. }
  446. func TestWatchEventType(t *testing.T) {
  447. cluster := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  448. defer cluster.Terminate(t)
  449. client := cluster.RandClient()
  450. ctx := context.Background()
  451. watchChan := client.Watch(ctx, "/", clientv3.WithPrefix())
  452. if _, err := client.Put(ctx, "/toDelete", "foo"); err != nil {
  453. t.Fatalf("Put failed: %v", err)
  454. }
  455. if _, err := client.Put(ctx, "/toDelete", "bar"); err != nil {
  456. t.Fatalf("Put failed: %v", err)
  457. }
  458. if _, err := client.Delete(ctx, "/toDelete"); err != nil {
  459. t.Fatalf("Delete failed: %v", err)
  460. }
  461. lcr, err := client.Lease.Grant(ctx, 1)
  462. if err != nil {
  463. t.Fatalf("lease create failed: %v", err)
  464. }
  465. if _, err := client.Put(ctx, "/toExpire", "foo", clientv3.WithLease(lcr.ID)); err != nil {
  466. t.Fatalf("Put failed: %v", err)
  467. }
  468. tests := []struct {
  469. et mvccpb.Event_EventType
  470. isCreate bool
  471. isModify bool
  472. }{{
  473. et: clientv3.EventTypePut,
  474. isCreate: true,
  475. }, {
  476. et: clientv3.EventTypePut,
  477. isModify: true,
  478. }, {
  479. et: clientv3.EventTypeDelete,
  480. }, {
  481. et: clientv3.EventTypePut,
  482. isCreate: true,
  483. }, {
  484. et: clientv3.EventTypeDelete,
  485. }}
  486. var res []*clientv3.Event
  487. for {
  488. select {
  489. case wres := <-watchChan:
  490. res = append(res, wres.Events...)
  491. case <-time.After(10 * time.Second):
  492. t.Fatalf("Should receive %d events and then break out loop", len(tests))
  493. }
  494. if len(res) == len(tests) {
  495. break
  496. }
  497. }
  498. for i, tt := range tests {
  499. ev := res[i]
  500. if tt.et != ev.Type {
  501. t.Errorf("#%d: event type want=%s, get=%s", i, tt.et, ev.Type)
  502. }
  503. if tt.isCreate && !ev.IsCreate() {
  504. t.Errorf("#%d: event should be CreateEvent", i)
  505. }
  506. if tt.isModify && !ev.IsModify() {
  507. t.Errorf("#%d: event should be ModifyEvent", i)
  508. }
  509. }
  510. }
  511. func TestWatchErrConnClosed(t *testing.T) {
  512. defer testutil.AfterTest(t)
  513. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  514. defer clus.Terminate(t)
  515. cli := clus.Client(0)
  516. wc := clientv3.NewWatcher(cli)
  517. donec := make(chan struct{})
  518. go func() {
  519. defer close(donec)
  520. wc.Watch(context.TODO(), "foo")
  521. if err := wc.Close(); err != nil && err != grpc.ErrClientConnClosing {
  522. t.Fatalf("expected %v, got %v", grpc.ErrClientConnClosing, err)
  523. }
  524. }()
  525. if err := cli.Close(); err != nil {
  526. t.Fatal(err)
  527. }
  528. clus.TakeClient(0)
  529. select {
  530. case <-time.After(3 * time.Second):
  531. t.Fatal("wc.Watch took too long")
  532. case <-donec:
  533. }
  534. }
  535. func TestWatchAfterClose(t *testing.T) {
  536. defer testutil.AfterTest(t)
  537. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  538. defer clus.Terminate(t)
  539. cli := clus.Client(0)
  540. clus.TakeClient(0)
  541. if err := cli.Close(); err != nil {
  542. t.Fatal(err)
  543. }
  544. donec := make(chan struct{})
  545. go func() {
  546. wc := clientv3.NewWatcher(cli)
  547. wc.Watch(context.TODO(), "foo")
  548. if err := wc.Close(); err != nil && err != grpc.ErrClientConnClosing {
  549. t.Fatalf("expected %v, got %v", grpc.ErrClientConnClosing, err)
  550. }
  551. close(donec)
  552. }()
  553. select {
  554. case <-time.After(3 * time.Second):
  555. t.Fatal("wc.Watch took too long")
  556. case <-donec:
  557. }
  558. }
  559. // TestWatchWithRequireLeader checks the watch channel closes when no leader.
  560. func TestWatchWithRequireLeader(t *testing.T) {
  561. defer testutil.AfterTest(t)
  562. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 3})
  563. defer clus.Terminate(t)
  564. // something for the non-require leader watch to read as an event
  565. if _, err := clus.Client(1).Put(context.TODO(), "foo", "bar"); err != nil {
  566. t.Fatal(err)
  567. }
  568. clus.Members[1].Stop(t)
  569. clus.Members[2].Stop(t)
  570. clus.Client(1).Close()
  571. clus.Client(2).Close()
  572. clus.TakeClient(1)
  573. clus.TakeClient(2)
  574. // wait for election timeout, then member[0] will not have a leader.
  575. tickDuration := 10 * time.Millisecond
  576. time.Sleep(time.Duration(3*clus.Members[0].ElectionTicks) * tickDuration)
  577. chLeader := clus.Client(0).Watch(clientv3.WithRequireLeader(context.TODO()), "foo", clientv3.WithRev(1))
  578. chNoLeader := clus.Client(0).Watch(context.TODO(), "foo", clientv3.WithRev(1))
  579. select {
  580. case resp, ok := <-chLeader:
  581. if !ok {
  582. t.Fatalf("expected %v watch channel, got closed channel", rpctypes.ErrNoLeader)
  583. }
  584. if resp.Err() != rpctypes.ErrNoLeader {
  585. t.Fatalf("expected %v watch response error, got %+v", rpctypes.ErrNoLeader, resp)
  586. }
  587. case <-time.After(3 * time.Second):
  588. t.Fatal("watch without leader took too long to close")
  589. }
  590. select {
  591. case resp, ok := <-chLeader:
  592. if ok {
  593. t.Fatalf("expected closed channel, got response %v", resp)
  594. }
  595. case <-time.After(3 * time.Second):
  596. t.Fatal("waited too long for channel to close")
  597. }
  598. if _, ok := <-chNoLeader; !ok {
  599. t.Fatalf("expected response, got closed channel")
  600. }
  601. }
  602. // TestWatchWithFilter checks that watch filtering works.
  603. func TestWatchWithFilter(t *testing.T) {
  604. cluster := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  605. defer cluster.Terminate(t)
  606. client := cluster.RandClient()
  607. ctx := context.Background()
  608. wcNoPut := client.Watch(ctx, "a", clientv3.WithFilterPut())
  609. wcNoDel := client.Watch(ctx, "a", clientv3.WithFilterDelete())
  610. if _, err := client.Put(ctx, "a", "abc"); err != nil {
  611. t.Fatal(err)
  612. }
  613. if _, err := client.Delete(ctx, "a"); err != nil {
  614. t.Fatal(err)
  615. }
  616. npResp := <-wcNoPut
  617. if len(npResp.Events) != 1 || npResp.Events[0].Type != clientv3.EventTypeDelete {
  618. t.Fatalf("expected delete event, got %+v", npResp.Events)
  619. }
  620. ndResp := <-wcNoDel
  621. if len(ndResp.Events) != 1 || ndResp.Events[0].Type != clientv3.EventTypePut {
  622. t.Fatalf("expected put event, got %+v", ndResp.Events)
  623. }
  624. select {
  625. case resp := <-wcNoPut:
  626. t.Fatalf("unexpected event on filtered put (%+v)", resp)
  627. case resp := <-wcNoDel:
  628. t.Fatalf("unexpected event on filtered delete (%+v)", resp)
  629. case <-time.After(100 * time.Millisecond):
  630. }
  631. }
  632. // TestWatchWithCreatedNotification checks that createdNotification works.
  633. func TestWatchWithCreatedNotification(t *testing.T) {
  634. cluster := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  635. defer cluster.Terminate(t)
  636. client := cluster.RandClient()
  637. ctx := context.Background()
  638. createC := client.Watch(ctx, "a", clientv3.WithCreatedNotify())
  639. resp := <-createC
  640. if !resp.Created {
  641. t.Fatalf("expected created event, got %v", resp)
  642. }
  643. }
  644. // TestWatchCancelOnServer ensures client watcher cancels propagate back to the server.
  645. func TestWatchCancelOnServer(t *testing.T) {
  646. cluster := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  647. defer cluster.Terminate(t)
  648. client := cluster.RandClient()
  649. for i := 0; i < 10; i++ {
  650. ctx, cancel := context.WithTimeout(context.Background(), time.Second)
  651. client.Watch(ctx, "a", clientv3.WithCreatedNotify())
  652. cancel()
  653. }
  654. // wait for cancels to propagate
  655. time.Sleep(time.Second)
  656. watchers, err := cluster.Members[0].Metric("etcd_debugging_mvcc_watcher_total")
  657. if err != nil {
  658. t.Fatal(err)
  659. }
  660. if watchers != "0" {
  661. t.Fatalf("expected 0 watchers, got %q", watchers)
  662. }
  663. }
  664. // TestWatchOverlapContextCancel stresses the watcher stream teardown path by
  665. // creating/canceling watchers to ensure that new watchers are not taken down
  666. // by a torn down watch stream. The sort of race that's being detected:
  667. // 1. create w1 using a cancelable ctx with %v as "ctx"
  668. // 2. cancel ctx
  669. // 3. watcher client begins tearing down watcher grpc stream since no more watchers
  670. // 3. start creating watcher w2 using a new "ctx" (not canceled), attaches to old grpc stream
  671. // 4. watcher client finishes tearing down stream on "ctx"
  672. // 5. w2 comes back canceled
  673. func TestWatchOverlapContextCancel(t *testing.T) {
  674. f := func(clus *integration.ClusterV3) {}
  675. testWatchOverlapContextCancel(t, f)
  676. }
  677. func TestWatchOverlapDropConnContextCancel(t *testing.T) {
  678. f := func(clus *integration.ClusterV3) {
  679. clus.Members[0].DropConnections()
  680. }
  681. testWatchOverlapContextCancel(t, f)
  682. }
  683. func testWatchOverlapContextCancel(t *testing.T, f func(*integration.ClusterV3)) {
  684. defer testutil.AfterTest(t)
  685. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  686. defer clus.Terminate(t)
  687. // each unique context "%v" has a unique grpc stream
  688. n := 100
  689. ctxs, ctxc := make([]context.Context, 5), make([]chan struct{}, 5)
  690. for i := range ctxs {
  691. // make "%v" unique
  692. ctxs[i] = context.WithValue(context.TODO(), "key", i)
  693. // limits the maximum number of outstanding watchers per stream
  694. ctxc[i] = make(chan struct{}, 2)
  695. }
  696. // issue concurrent watches on "abc" with cancel
  697. cli := clus.RandClient()
  698. if _, err := cli.Put(context.TODO(), "abc", "def"); err != nil {
  699. t.Fatal(err)
  700. }
  701. ch := make(chan struct{}, n)
  702. for i := 0; i < n; i++ {
  703. go func() {
  704. defer func() { ch <- struct{}{} }()
  705. idx := rand.Intn(len(ctxs))
  706. ctx, cancel := context.WithCancel(ctxs[idx])
  707. ctxc[idx] <- struct{}{}
  708. wch := cli.Watch(ctx, "abc", clientv3.WithRev(1))
  709. f(clus)
  710. select {
  711. case _, ok := <-wch:
  712. if !ok {
  713. t.Fatalf("unexpected closed channel %p", wch)
  714. }
  715. // may take a second or two to reestablish a watcher because of
  716. // grpc backoff policies for disconnects
  717. case <-time.After(5 * time.Second):
  718. t.Errorf("timed out waiting for watch on %p", wch)
  719. }
  720. // randomize how cancel overlaps with watch creation
  721. if rand.Intn(2) == 0 {
  722. <-ctxc[idx]
  723. cancel()
  724. } else {
  725. cancel()
  726. <-ctxc[idx]
  727. }
  728. }()
  729. }
  730. // join on watches
  731. for i := 0; i < n; i++ {
  732. select {
  733. case <-ch:
  734. case <-time.After(5 * time.Second):
  735. t.Fatalf("timed out waiting for completed watch")
  736. }
  737. }
  738. }
  739. // TestWatchCanelAndCloseClient ensures that canceling a watcher then immediately
  740. // closing the client does not return a client closing error.
  741. func TestWatchCancelAndCloseClient(t *testing.T) {
  742. defer testutil.AfterTest(t)
  743. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  744. defer clus.Terminate(t)
  745. cli := clus.Client(0)
  746. ctx, cancel := context.WithCancel(context.Background())
  747. wch := cli.Watch(ctx, "abc")
  748. donec := make(chan struct{})
  749. go func() {
  750. defer close(donec)
  751. select {
  752. case wr, ok := <-wch:
  753. if ok {
  754. t.Fatalf("expected closed watch after cancel(), got resp=%+v err=%v", wr, wr.Err())
  755. }
  756. case <-time.After(5 * time.Second):
  757. t.Fatal("timed out waiting for closed channel")
  758. }
  759. }()
  760. cancel()
  761. if err := cli.Close(); err != nil {
  762. t.Fatal(err)
  763. }
  764. <-donec
  765. clus.TakeClient(0)
  766. }
  767. // TestWatchStressResumeClose establishes a bunch of watchers, disconnects
  768. // to put them in resuming mode, cancels them so some resumes by cancel fail,
  769. // then closes the watcher interface to ensure correct clean up.
  770. func TestWatchStressResumeClose(t *testing.T) {
  771. defer testutil.AfterTest(t)
  772. clus := integration.NewClusterV3(t, &integration.ClusterConfig{Size: 1})
  773. defer clus.Terminate(t)
  774. cli := clus.Client(0)
  775. ctx, cancel := context.WithCancel(context.Background())
  776. // add more watches than can be resumed before the cancel
  777. wchs := make([]clientv3.WatchChan, 2000)
  778. for i := range wchs {
  779. wchs[i] = cli.Watch(ctx, "abc")
  780. }
  781. clus.Members[0].DropConnections()
  782. cancel()
  783. if err := cli.Close(); err != nil {
  784. t.Fatal(err)
  785. }
  786. clus.TakeClient(0)
  787. }