kvstore_test.go 27 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844845846847848849850851852853854855856857858859860861862863864865866867868869870871872873874875876877878879880881882883884885886887888889890891892893894895896897898899900901902903904905906907908909910911912913914915916917918919920921922923924925926927928929930931932933934935936937938939940941942943944945946947948949950951952953954955956957958959960961962963964965966967968969970971972973974975976977978979980981982983984985986987988989990991992993994995
  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. "bytes"
  17. "crypto/rand"
  18. "encoding/binary"
  19. "fmt"
  20. "math"
  21. mrand "math/rand"
  22. "os"
  23. "reflect"
  24. "sort"
  25. "strconv"
  26. "sync"
  27. "testing"
  28. "time"
  29. "go.etcd.io/etcd/lease"
  30. "go.etcd.io/etcd/mvcc/backend"
  31. "go.etcd.io/etcd/mvcc/mvccpb"
  32. "go.etcd.io/etcd/pkg/schedule"
  33. "go.etcd.io/etcd/pkg/testutil"
  34. "go.etcd.io/etcd/pkg/traceutil"
  35. "go.uber.org/zap"
  36. )
  37. func TestStoreRev(t *testing.T) {
  38. b, tmpPath := backend.NewDefaultTmpBackend()
  39. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  40. defer s.Close()
  41. defer os.Remove(tmpPath)
  42. for i := 1; i <= 3; i++ {
  43. s.Put([]byte("foo"), []byte("bar"), lease.NoLease)
  44. if r := s.Rev(); r != int64(i+1) {
  45. t.Errorf("#%d: rev = %d, want %d", i, r, i+1)
  46. }
  47. }
  48. }
  49. func TestStorePut(t *testing.T) {
  50. kv := mvccpb.KeyValue{
  51. Key: []byte("foo"),
  52. Value: []byte("bar"),
  53. CreateRevision: 1,
  54. ModRevision: 2,
  55. Version: 1,
  56. }
  57. kvb, err := kv.Marshal()
  58. if err != nil {
  59. t.Fatal(err)
  60. }
  61. tests := []struct {
  62. rev revision
  63. r indexGetResp
  64. rr *rangeResp
  65. wrev revision
  66. wkey []byte
  67. wkv mvccpb.KeyValue
  68. wputrev revision
  69. }{
  70. {
  71. revision{1, 0},
  72. indexGetResp{revision{}, revision{}, 0, ErrRevisionNotFound},
  73. nil,
  74. revision{2, 0},
  75. newTestKeyBytes(revision{2, 0}, false),
  76. mvccpb.KeyValue{
  77. Key: []byte("foo"),
  78. Value: []byte("bar"),
  79. CreateRevision: 2,
  80. ModRevision: 2,
  81. Version: 1,
  82. Lease: 1,
  83. },
  84. revision{2, 0},
  85. },
  86. {
  87. revision{1, 1},
  88. indexGetResp{revision{2, 0}, revision{2, 0}, 1, nil},
  89. &rangeResp{[][]byte{newTestKeyBytes(revision{2, 1}, false)}, [][]byte{kvb}},
  90. revision{2, 0},
  91. newTestKeyBytes(revision{2, 0}, false),
  92. mvccpb.KeyValue{
  93. Key: []byte("foo"),
  94. Value: []byte("bar"),
  95. CreateRevision: 2,
  96. ModRevision: 2,
  97. Version: 2,
  98. Lease: 2,
  99. },
  100. revision{2, 0},
  101. },
  102. {
  103. revision{2, 0},
  104. indexGetResp{revision{2, 1}, revision{2, 0}, 2, nil},
  105. &rangeResp{[][]byte{newTestKeyBytes(revision{2, 1}, false)}, [][]byte{kvb}},
  106. revision{3, 0},
  107. newTestKeyBytes(revision{3, 0}, false),
  108. mvccpb.KeyValue{
  109. Key: []byte("foo"),
  110. Value: []byte("bar"),
  111. CreateRevision: 2,
  112. ModRevision: 3,
  113. Version: 3,
  114. Lease: 3,
  115. },
  116. revision{3, 0},
  117. },
  118. }
  119. for i, tt := range tests {
  120. s := newFakeStore()
  121. b := s.b.(*fakeBackend)
  122. fi := s.kvindex.(*fakeIndex)
  123. s.currentRev = tt.rev.main
  124. fi.indexGetRespc <- tt.r
  125. if tt.rr != nil {
  126. b.tx.rangeRespc <- *tt.rr
  127. }
  128. s.Put([]byte("foo"), []byte("bar"), lease.LeaseID(i+1))
  129. data, err := tt.wkv.Marshal()
  130. if err != nil {
  131. t.Errorf("#%d: marshal err = %v, want nil", i, err)
  132. }
  133. wact := []testutil.Action{
  134. {Name: "seqput", Params: []interface{}{keyBucketName, tt.wkey, data}},
  135. }
  136. if tt.rr != nil {
  137. wact = []testutil.Action{
  138. {Name: "seqput", Params: []interface{}{keyBucketName, tt.wkey, data}},
  139. }
  140. }
  141. if g := b.tx.Action(); !reflect.DeepEqual(g, wact) {
  142. t.Errorf("#%d: tx action = %+v, want %+v", i, g, wact)
  143. }
  144. wact = []testutil.Action{
  145. {Name: "get", Params: []interface{}{[]byte("foo"), tt.wputrev.main}},
  146. {Name: "put", Params: []interface{}{[]byte("foo"), tt.wputrev}},
  147. }
  148. if g := fi.Action(); !reflect.DeepEqual(g, wact) {
  149. t.Errorf("#%d: index action = %+v, want %+v", i, g, wact)
  150. }
  151. if s.currentRev != tt.wrev.main {
  152. t.Errorf("#%d: rev = %+v, want %+v", i, s.currentRev, tt.wrev)
  153. }
  154. s.Close()
  155. }
  156. }
  157. func TestStoreRange(t *testing.T) {
  158. key := newTestKeyBytes(revision{2, 0}, false)
  159. kv := mvccpb.KeyValue{
  160. Key: []byte("foo"),
  161. Value: []byte("bar"),
  162. CreateRevision: 1,
  163. ModRevision: 2,
  164. Version: 1,
  165. }
  166. kvb, err := kv.Marshal()
  167. if err != nil {
  168. t.Fatal(err)
  169. }
  170. wrev := int64(2)
  171. tests := []struct {
  172. idxr indexRangeResp
  173. r rangeResp
  174. }{
  175. {
  176. indexRangeResp{[][]byte{[]byte("foo")}, []revision{{2, 0}}},
  177. rangeResp{[][]byte{key}, [][]byte{kvb}},
  178. },
  179. {
  180. indexRangeResp{[][]byte{[]byte("foo"), []byte("foo1")}, []revision{{2, 0}, {3, 0}}},
  181. rangeResp{[][]byte{key}, [][]byte{kvb}},
  182. },
  183. }
  184. ro := RangeOptions{Limit: 1, Rev: 0, Count: false}
  185. for i, tt := range tests {
  186. s := newFakeStore()
  187. b := s.b.(*fakeBackend)
  188. fi := s.kvindex.(*fakeIndex)
  189. s.currentRev = 2
  190. b.tx.rangeRespc <- tt.r
  191. fi.indexRangeRespc <- tt.idxr
  192. ret, err := s.Range([]byte("foo"), []byte("goo"), ro)
  193. if err != nil {
  194. t.Errorf("#%d: err = %v, want nil", i, err)
  195. }
  196. if w := []mvccpb.KeyValue{kv}; !reflect.DeepEqual(ret.KVs, w) {
  197. t.Errorf("#%d: kvs = %+v, want %+v", i, ret.KVs, w)
  198. }
  199. if ret.Rev != wrev {
  200. t.Errorf("#%d: rev = %d, want %d", i, ret.Rev, wrev)
  201. }
  202. wstart := newRevBytes()
  203. revToBytes(tt.idxr.revs[0], wstart)
  204. wact := []testutil.Action{
  205. {Name: "range", Params: []interface{}{keyBucketName, wstart, []byte(nil), int64(0)}},
  206. }
  207. if g := b.tx.Action(); !reflect.DeepEqual(g, wact) {
  208. t.Errorf("#%d: tx action = %+v, want %+v", i, g, wact)
  209. }
  210. wact = []testutil.Action{
  211. {Name: "range", Params: []interface{}{[]byte("foo"), []byte("goo"), wrev}},
  212. }
  213. if g := fi.Action(); !reflect.DeepEqual(g, wact) {
  214. t.Errorf("#%d: index action = %+v, want %+v", i, g, wact)
  215. }
  216. if s.currentRev != 2 {
  217. t.Errorf("#%d: current rev = %+v, want %+v", i, s.currentRev, 2)
  218. }
  219. s.Close()
  220. }
  221. }
  222. func TestStoreDeleteRange(t *testing.T) {
  223. key := newTestKeyBytes(revision{2, 0}, false)
  224. kv := mvccpb.KeyValue{
  225. Key: []byte("foo"),
  226. Value: []byte("bar"),
  227. CreateRevision: 1,
  228. ModRevision: 2,
  229. Version: 1,
  230. }
  231. kvb, err := kv.Marshal()
  232. if err != nil {
  233. t.Fatal(err)
  234. }
  235. tests := []struct {
  236. rev revision
  237. r indexRangeResp
  238. rr rangeResp
  239. wkey []byte
  240. wrev revision
  241. wrrev int64
  242. wdelrev revision
  243. }{
  244. {
  245. revision{2, 0},
  246. indexRangeResp{[][]byte{[]byte("foo")}, []revision{{2, 0}}},
  247. rangeResp{[][]byte{key}, [][]byte{kvb}},
  248. newTestKeyBytes(revision{3, 0}, true),
  249. revision{3, 0},
  250. 2,
  251. revision{3, 0},
  252. },
  253. }
  254. for i, tt := range tests {
  255. s := newFakeStore()
  256. b := s.b.(*fakeBackend)
  257. fi := s.kvindex.(*fakeIndex)
  258. s.currentRev = tt.rev.main
  259. fi.indexRangeRespc <- tt.r
  260. b.tx.rangeRespc <- tt.rr
  261. n, _ := s.DeleteRange([]byte("foo"), []byte("goo"))
  262. if n != 1 {
  263. t.Errorf("#%d: n = %d, want 1", i, n)
  264. }
  265. data, err := (&mvccpb.KeyValue{
  266. Key: []byte("foo"),
  267. }).Marshal()
  268. if err != nil {
  269. t.Errorf("#%d: marshal err = %v, want nil", i, err)
  270. }
  271. wact := []testutil.Action{
  272. {Name: "seqput", Params: []interface{}{keyBucketName, tt.wkey, data}},
  273. }
  274. if g := b.tx.Action(); !reflect.DeepEqual(g, wact) {
  275. t.Errorf("#%d: tx action = %+v, want %+v", i, g, wact)
  276. }
  277. wact = []testutil.Action{
  278. {Name: "range", Params: []interface{}{[]byte("foo"), []byte("goo"), tt.wrrev}},
  279. {Name: "tombstone", Params: []interface{}{[]byte("foo"), tt.wdelrev}},
  280. }
  281. if g := fi.Action(); !reflect.DeepEqual(g, wact) {
  282. t.Errorf("#%d: index action = %+v, want %+v", i, g, wact)
  283. }
  284. if s.currentRev != tt.wrev.main {
  285. t.Errorf("#%d: rev = %+v, want %+v", i, s.currentRev, tt.wrev)
  286. }
  287. }
  288. }
  289. func TestStoreCompact(t *testing.T) {
  290. s := newFakeStore()
  291. defer s.Close()
  292. b := s.b.(*fakeBackend)
  293. fi := s.kvindex.(*fakeIndex)
  294. s.currentRev = 3
  295. fi.indexCompactRespc <- map[revision]struct{}{{1, 0}: {}}
  296. key1 := newTestKeyBytes(revision{1, 0}, false)
  297. key2 := newTestKeyBytes(revision{2, 0}, false)
  298. b.tx.rangeRespc <- rangeResp{[][]byte{key1, key2}, nil}
  299. s.Compact(traceutil.TODO(), 3)
  300. s.fifoSched.WaitFinish(1)
  301. if s.compactMainRev != 3 {
  302. t.Errorf("compact main rev = %d, want 3", s.compactMainRev)
  303. }
  304. end := make([]byte, 8)
  305. binary.BigEndian.PutUint64(end, uint64(4))
  306. wact := []testutil.Action{
  307. {Name: "put", Params: []interface{}{metaBucketName, scheduledCompactKeyName, newTestRevBytes(revision{3, 0})}},
  308. {Name: "range", Params: []interface{}{keyBucketName, make([]byte, 17), end, int64(10000)}},
  309. {Name: "delete", Params: []interface{}{keyBucketName, key2}},
  310. {Name: "put", Params: []interface{}{metaBucketName, finishedCompactKeyName, newTestRevBytes(revision{3, 0})}},
  311. }
  312. if g := b.tx.Action(); !reflect.DeepEqual(g, wact) {
  313. t.Errorf("tx actions = %+v, want %+v", g, wact)
  314. }
  315. wact = []testutil.Action{
  316. {Name: "compact", Params: []interface{}{int64(3)}},
  317. }
  318. if g := fi.Action(); !reflect.DeepEqual(g, wact) {
  319. t.Errorf("index action = %+v, want %+v", g, wact)
  320. }
  321. }
  322. func TestStoreRestore(t *testing.T) {
  323. s := newFakeStore()
  324. b := s.b.(*fakeBackend)
  325. fi := s.kvindex.(*fakeIndex)
  326. putkey := newTestKeyBytes(revision{3, 0}, false)
  327. putkv := mvccpb.KeyValue{
  328. Key: []byte("foo"),
  329. Value: []byte("bar"),
  330. CreateRevision: 4,
  331. ModRevision: 4,
  332. Version: 1,
  333. }
  334. putkvb, err := putkv.Marshal()
  335. if err != nil {
  336. t.Fatal(err)
  337. }
  338. delkey := newTestKeyBytes(revision{5, 0}, true)
  339. delkv := mvccpb.KeyValue{
  340. Key: []byte("foo"),
  341. }
  342. delkvb, err := delkv.Marshal()
  343. if err != nil {
  344. t.Fatal(err)
  345. }
  346. b.tx.rangeRespc <- rangeResp{[][]byte{finishedCompactKeyName}, [][]byte{newTestRevBytes(revision{3, 0})}}
  347. b.tx.rangeRespc <- rangeResp{[][]byte{scheduledCompactKeyName}, [][]byte{newTestRevBytes(revision{3, 0})}}
  348. b.tx.rangeRespc <- rangeResp{[][]byte{putkey, delkey}, [][]byte{putkvb, delkvb}}
  349. b.tx.rangeRespc <- rangeResp{nil, nil}
  350. s.restore()
  351. if s.compactMainRev != 3 {
  352. t.Errorf("compact rev = %d, want 3", s.compactMainRev)
  353. }
  354. if s.currentRev != 5 {
  355. t.Errorf("current rev = %v, want 5", s.currentRev)
  356. }
  357. wact := []testutil.Action{
  358. {Name: "range", Params: []interface{}{metaBucketName, finishedCompactKeyName, []byte(nil), int64(0)}},
  359. {Name: "range", Params: []interface{}{metaBucketName, scheduledCompactKeyName, []byte(nil), int64(0)}},
  360. {Name: "range", Params: []interface{}{keyBucketName, newTestRevBytes(revision{1, 0}), newTestRevBytes(revision{math.MaxInt64, math.MaxInt64}), int64(restoreChunkKeys)}},
  361. }
  362. if g := b.tx.Action(); !reflect.DeepEqual(g, wact) {
  363. t.Errorf("tx actions = %+v, want %+v", g, wact)
  364. }
  365. gens := []generation{
  366. {created: revision{4, 0}, ver: 2, revs: []revision{{3, 0}, {5, 0}}},
  367. {created: revision{0, 0}, ver: 0, revs: nil},
  368. }
  369. ki := &keyIndex{key: []byte("foo"), modified: revision{5, 0}, generations: gens}
  370. wact = []testutil.Action{
  371. {Name: "keyIndex", Params: []interface{}{ki}},
  372. {Name: "insert", Params: []interface{}{ki}},
  373. }
  374. if g := fi.Action(); !reflect.DeepEqual(g, wact) {
  375. t.Errorf("index action = %+v, want %+v", g, wact)
  376. }
  377. }
  378. func TestRestoreDelete(t *testing.T) {
  379. oldChunk := restoreChunkKeys
  380. restoreChunkKeys = mrand.Intn(3) + 2
  381. defer func() { restoreChunkKeys = oldChunk }()
  382. b, tmpPath := backend.NewDefaultTmpBackend()
  383. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  384. defer os.Remove(tmpPath)
  385. keys := make(map[string]struct{})
  386. for i := 0; i < 20; i++ {
  387. ks := fmt.Sprintf("foo-%d", i)
  388. k := []byte(ks)
  389. s.Put(k, []byte("bar"), lease.NoLease)
  390. keys[ks] = struct{}{}
  391. switch mrand.Intn(3) {
  392. case 0:
  393. // put random key from past via random range on map
  394. ks = fmt.Sprintf("foo-%d", mrand.Intn(i+1))
  395. s.Put([]byte(ks), []byte("baz"), lease.NoLease)
  396. keys[ks] = struct{}{}
  397. case 1:
  398. // delete random key via random range on map
  399. for k := range keys {
  400. s.DeleteRange([]byte(k), nil)
  401. delete(keys, k)
  402. break
  403. }
  404. }
  405. }
  406. s.Close()
  407. s = NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  408. defer s.Close()
  409. for i := 0; i < 20; i++ {
  410. ks := fmt.Sprintf("foo-%d", i)
  411. r, err := s.Range([]byte(ks), nil, RangeOptions{})
  412. if err != nil {
  413. t.Fatal(err)
  414. }
  415. if _, ok := keys[ks]; ok {
  416. if len(r.KVs) == 0 {
  417. t.Errorf("#%d: expected %q, got deleted", i, ks)
  418. }
  419. } else if len(r.KVs) != 0 {
  420. t.Errorf("#%d: expected deleted, got %q", i, ks)
  421. }
  422. }
  423. }
  424. func TestRestoreContinueUnfinishedCompaction(t *testing.T) {
  425. tests := []string{"recreate", "restore"}
  426. for _, test := range tests {
  427. b, tmpPath := backend.NewDefaultTmpBackend()
  428. s0 := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  429. defer os.Remove(tmpPath)
  430. s0.Put([]byte("foo"), []byte("bar"), lease.NoLease)
  431. s0.Put([]byte("foo"), []byte("bar1"), lease.NoLease)
  432. s0.Put([]byte("foo"), []byte("bar2"), lease.NoLease)
  433. // write scheduled compaction, but not do compaction
  434. rbytes := newRevBytes()
  435. revToBytes(revision{main: 2}, rbytes)
  436. tx := s0.b.BatchTx()
  437. tx.Lock()
  438. tx.UnsafePut(metaBucketName, scheduledCompactKeyName, rbytes)
  439. tx.Unlock()
  440. s0.Close()
  441. var s *store
  442. switch test {
  443. case "recreate":
  444. s = NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  445. case "restore":
  446. s0.Restore(b)
  447. s = s0
  448. }
  449. // wait for scheduled compaction to be finished
  450. time.Sleep(100 * time.Millisecond)
  451. if _, err := s.Range([]byte("foo"), nil, RangeOptions{Rev: 1}); err != ErrCompacted {
  452. t.Errorf("range on compacted rev error = %v, want %v", err, ErrCompacted)
  453. }
  454. // check the key in backend is deleted
  455. revbytes := newRevBytes()
  456. revToBytes(revision{main: 1}, revbytes)
  457. // The disk compaction is done asynchronously and requires more time on slow disk.
  458. // try 5 times for CI with slow IO.
  459. for i := 0; i < 5; i++ {
  460. tx = s.b.BatchTx()
  461. tx.Lock()
  462. ks, _ := tx.UnsafeRange(keyBucketName, revbytes, nil, 0)
  463. tx.Unlock()
  464. if len(ks) != 0 {
  465. time.Sleep(100 * time.Millisecond)
  466. continue
  467. }
  468. return
  469. }
  470. t.Errorf("key for rev %+v still exists, want deleted", bytesToRev(revbytes))
  471. }
  472. }
  473. type hashKVResult struct {
  474. hash uint32
  475. compactRev int64
  476. }
  477. // TestHashKVWhenCompacting ensures that HashKV returns correct hash when compacting.
  478. func TestHashKVWhenCompacting(t *testing.T) {
  479. b, tmpPath := backend.NewDefaultTmpBackend()
  480. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  481. defer os.Remove(tmpPath)
  482. rev := 10000
  483. for i := 2; i <= rev; i++ {
  484. s.Put([]byte("foo"), []byte(fmt.Sprintf("bar%d", i)), lease.NoLease)
  485. }
  486. hashCompactc := make(chan hashKVResult, 1)
  487. donec := make(chan struct{})
  488. var wg sync.WaitGroup
  489. for i := 0; i < 10; i++ {
  490. wg.Add(1)
  491. go func() {
  492. defer wg.Done()
  493. for {
  494. hash, _, compactRev, err := s.HashByRev(int64(rev))
  495. if err != nil {
  496. t.Error(err)
  497. }
  498. select {
  499. case <-donec:
  500. return
  501. case hashCompactc <- hashKVResult{hash, compactRev}:
  502. }
  503. }
  504. }()
  505. }
  506. go func() {
  507. defer close(donec)
  508. revHash := make(map[int64]uint32)
  509. for round := 0; round < 1000; round++ {
  510. r := <-hashCompactc
  511. if revHash[r.compactRev] == 0 {
  512. revHash[r.compactRev] = r.hash
  513. }
  514. if r.hash != revHash[r.compactRev] {
  515. t.Errorf("Hashes differ (current %v) != (saved %v)", r.hash, revHash[r.compactRev])
  516. }
  517. }
  518. }()
  519. wg.Add(1)
  520. go func() {
  521. defer wg.Done()
  522. for i := 100; i >= 0; i-- {
  523. _, err := s.Compact(traceutil.TODO(), int64(rev-1-i))
  524. if err != nil {
  525. t.Error(err)
  526. }
  527. time.Sleep(10 * time.Millisecond)
  528. }
  529. }()
  530. select {
  531. case <-donec:
  532. wg.Wait()
  533. case <-time.After(10 * time.Second):
  534. testutil.FatalStack(t, "timeout")
  535. }
  536. }
  537. // TestHashKVZeroRevision ensures that "HashByRev(0)" computes
  538. // correct hash value with latest revision.
  539. func TestHashKVZeroRevision(t *testing.T) {
  540. b, tmpPath := backend.NewDefaultTmpBackend()
  541. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  542. defer os.Remove(tmpPath)
  543. rev := 10000
  544. for i := 2; i <= rev; i++ {
  545. s.Put([]byte("foo"), []byte(fmt.Sprintf("bar%d", i)), lease.NoLease)
  546. }
  547. if _, err := s.Compact(traceutil.TODO(), int64(rev/2)); err != nil {
  548. t.Fatal(err)
  549. }
  550. hash1, _, _, err := s.HashByRev(int64(rev))
  551. if err != nil {
  552. t.Fatal(err)
  553. }
  554. var hash2 uint32
  555. hash2, _, _, err = s.HashByRev(0)
  556. if err != nil {
  557. t.Fatal(err)
  558. }
  559. if hash1 != hash2 {
  560. t.Errorf("hash %d (rev %d) != hash %d (rev 0)", hash1, rev, hash2)
  561. }
  562. }
  563. func TestTxnPut(t *testing.T) {
  564. // assign arbitrary size
  565. bytesN := 30
  566. sliceN := 100
  567. keys := createBytesSlice(bytesN, sliceN)
  568. vals := createBytesSlice(bytesN, sliceN)
  569. b, tmpPath := backend.NewDefaultTmpBackend()
  570. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  571. defer cleanup(s, b, tmpPath)
  572. for i := 0; i < sliceN; i++ {
  573. txn := s.Write(traceutil.TODO())
  574. base := int64(i + 2)
  575. if rev := txn.Put(keys[i], vals[i], lease.NoLease); rev != base {
  576. t.Errorf("#%d: rev = %d, want %d", i, rev, base)
  577. }
  578. txn.End()
  579. }
  580. }
  581. // TestConcurrentReadNotBlockingWrite ensures Read does not blocking Write after its creation
  582. func TestConcurrentReadNotBlockingWrite(t *testing.T) {
  583. b, tmpPath := backend.NewDefaultTmpBackend()
  584. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  585. defer os.Remove(tmpPath)
  586. // write something to read later
  587. s.Put([]byte("foo"), []byte("bar"), lease.NoLease)
  588. // readTx simulates a long read request
  589. readTx1 := s.Read(traceutil.TODO())
  590. // write should not be blocked by reads
  591. done := make(chan struct{})
  592. go func() {
  593. s.Put([]byte("foo"), []byte("newBar"), lease.NoLease) // this is a write Txn
  594. done <- struct{}{}
  595. }()
  596. select {
  597. case <-done:
  598. case <-time.After(1 * time.Second):
  599. t.Fatalf("write should not be blocked by read")
  600. }
  601. // readTx2 simulates a short read request
  602. readTx2 := s.Read(traceutil.TODO())
  603. ro := RangeOptions{Limit: 1, Rev: 0, Count: false}
  604. ret, err := readTx2.Range([]byte("foo"), nil, ro)
  605. if err != nil {
  606. t.Fatalf("failed to range: %v", err)
  607. }
  608. // readTx2 should see the result of new write
  609. w := mvccpb.KeyValue{
  610. Key: []byte("foo"),
  611. Value: []byte("newBar"),
  612. CreateRevision: 2,
  613. ModRevision: 3,
  614. Version: 2,
  615. }
  616. if !reflect.DeepEqual(ret.KVs[0], w) {
  617. t.Fatalf("range result = %+v, want = %+v", ret.KVs[0], w)
  618. }
  619. readTx2.End()
  620. ret, err = readTx1.Range([]byte("foo"), nil, ro)
  621. if err != nil {
  622. t.Fatalf("failed to range: %v", err)
  623. }
  624. // readTx1 should not see the result of new write
  625. w = mvccpb.KeyValue{
  626. Key: []byte("foo"),
  627. Value: []byte("bar"),
  628. CreateRevision: 2,
  629. ModRevision: 2,
  630. Version: 1,
  631. }
  632. if !reflect.DeepEqual(ret.KVs[0], w) {
  633. t.Fatalf("range result = %+v, want = %+v", ret.KVs[0], w)
  634. }
  635. readTx1.End()
  636. }
  637. // TestConcurrentReadTxAndWrite creates random concurrent Reads and Writes, and ensures Reads always see latest Writes
  638. func TestConcurrentReadTxAndWrite(t *testing.T) {
  639. var (
  640. numOfReads = 100
  641. numOfWrites = 100
  642. maxNumOfPutsPerWrite = 10
  643. committedKVs kvs // committedKVs records the key-value pairs written by the finished Write Txns
  644. mu sync.Mutex // mu protectes committedKVs
  645. )
  646. b, tmpPath := backend.NewDefaultTmpBackend()
  647. s := NewStore(zap.NewExample(), b, &lease.FakeLessor{}, nil, StoreConfig{})
  648. defer os.Remove(tmpPath)
  649. var wg sync.WaitGroup
  650. wg.Add(numOfWrites)
  651. for i := 0; i < numOfWrites; i++ {
  652. go func() {
  653. defer wg.Done()
  654. time.Sleep(time.Duration(mrand.Intn(100)) * time.Millisecond) // random starting time
  655. tx := s.Write(traceutil.TODO())
  656. numOfPuts := mrand.Intn(maxNumOfPutsPerWrite) + 1
  657. var pendingKvs kvs
  658. for j := 0; j < numOfPuts; j++ {
  659. k := []byte(strconv.Itoa(mrand.Int()))
  660. v := []byte(strconv.Itoa(mrand.Int()))
  661. tx.Put(k, v, lease.NoLease)
  662. pendingKvs = append(pendingKvs, kv{k, v})
  663. }
  664. // reads should not see above Puts until write is finished
  665. mu.Lock()
  666. committedKVs = merge(committedKVs, pendingKvs) // update shared data structure
  667. tx.End()
  668. mu.Unlock()
  669. }()
  670. }
  671. wg.Add(numOfReads)
  672. for i := 0; i < numOfReads; i++ {
  673. go func() {
  674. defer wg.Done()
  675. time.Sleep(time.Duration(mrand.Intn(100)) * time.Millisecond) // random starting time
  676. mu.Lock()
  677. wKVs := make(kvs, len(committedKVs))
  678. copy(wKVs, committedKVs)
  679. tx := s.Read(traceutil.TODO())
  680. mu.Unlock()
  681. // get all keys in backend store, and compare with wKVs
  682. ret, err := tx.Range([]byte("\x00000000"), []byte("\xffffffff"), RangeOptions{})
  683. tx.End()
  684. if err != nil {
  685. t.Errorf("failed to range keys: %v", err)
  686. return
  687. }
  688. if len(wKVs) == 0 && len(ret.KVs) == 0 { // no committed KVs yet
  689. return
  690. }
  691. var result kvs
  692. for _, keyValue := range ret.KVs {
  693. result = append(result, kv{keyValue.Key, keyValue.Value})
  694. }
  695. if !reflect.DeepEqual(wKVs, result) {
  696. t.Errorf("unexpected range result") // too many key value pairs, skip printing them
  697. }
  698. }()
  699. }
  700. // wait until go routines finish or timeout
  701. doneC := make(chan struct{})
  702. go func() {
  703. wg.Wait()
  704. close(doneC)
  705. }()
  706. select {
  707. case <-doneC:
  708. case <-time.After(5 * time.Minute):
  709. testutil.FatalStack(t, "timeout")
  710. }
  711. }
  712. type kv struct {
  713. key []byte
  714. val []byte
  715. }
  716. type kvs []kv
  717. func (kvs kvs) Len() int { return len(kvs) }
  718. func (kvs kvs) Less(i, j int) bool { return bytes.Compare(kvs[i].key, kvs[j].key) < 0 }
  719. func (kvs kvs) Swap(i, j int) { kvs[i], kvs[j] = kvs[j], kvs[i] }
  720. func merge(dst, src kvs) kvs {
  721. dst = append(dst, src...)
  722. sort.Stable(dst)
  723. // remove duplicates, using only the newest value
  724. // ref: tx_buffer.go
  725. widx := 0
  726. for ridx := 1; ridx < len(dst); ridx++ {
  727. if !bytes.Equal(dst[widx].key, dst[ridx].key) {
  728. widx++
  729. }
  730. dst[widx] = dst[ridx]
  731. }
  732. return dst[:widx+1]
  733. }
  734. // TODO: test attach key to lessor
  735. func newTestRevBytes(rev revision) []byte {
  736. bytes := newRevBytes()
  737. revToBytes(rev, bytes)
  738. return bytes
  739. }
  740. func newTestKeyBytes(rev revision, tombstone bool) []byte {
  741. bytes := newRevBytes()
  742. revToBytes(rev, bytes)
  743. if tombstone {
  744. bytes = appendMarkTombstone(zap.NewExample(), bytes)
  745. }
  746. return bytes
  747. }
  748. func newFakeStore() *store {
  749. b := &fakeBackend{&fakeBatchTx{
  750. Recorder: &testutil.RecorderBuffered{},
  751. rangeRespc: make(chan rangeResp, 5)}}
  752. fi := &fakeIndex{
  753. Recorder: &testutil.RecorderBuffered{},
  754. indexGetRespc: make(chan indexGetResp, 1),
  755. indexRangeRespc: make(chan indexRangeResp, 1),
  756. indexRangeEventsRespc: make(chan indexRangeEventsResp, 1),
  757. indexCompactRespc: make(chan map[revision]struct{}, 1),
  758. }
  759. s := &store{
  760. cfg: StoreConfig{CompactionBatchLimit: 10000},
  761. b: b,
  762. le: &lease.FakeLessor{},
  763. kvindex: fi,
  764. currentRev: 0,
  765. compactMainRev: -1,
  766. fifoSched: schedule.NewFIFOScheduler(),
  767. stopc: make(chan struct{}),
  768. lg: zap.NewExample(),
  769. }
  770. s.ReadView, s.WriteView = &readView{s}, &writeView{s}
  771. return s
  772. }
  773. type rangeResp struct {
  774. keys [][]byte
  775. vals [][]byte
  776. }
  777. type fakeBatchTx struct {
  778. testutil.Recorder
  779. rangeRespc chan rangeResp
  780. }
  781. func (b *fakeBatchTx) Lock() {}
  782. func (b *fakeBatchTx) Unlock() {}
  783. func (b *fakeBatchTx) RLock() {}
  784. func (b *fakeBatchTx) RUnlock() {}
  785. func (b *fakeBatchTx) UnsafeCreateBucket(name []byte) {}
  786. func (b *fakeBatchTx) UnsafePut(bucketName []byte, key []byte, value []byte) {
  787. b.Recorder.Record(testutil.Action{Name: "put", Params: []interface{}{bucketName, key, value}})
  788. }
  789. func (b *fakeBatchTx) UnsafeSeqPut(bucketName []byte, key []byte, value []byte) {
  790. b.Recorder.Record(testutil.Action{Name: "seqput", Params: []interface{}{bucketName, key, value}})
  791. }
  792. func (b *fakeBatchTx) UnsafeRange(bucketName []byte, key, endKey []byte, limit int64) (keys [][]byte, vals [][]byte) {
  793. b.Recorder.Record(testutil.Action{Name: "range", Params: []interface{}{bucketName, key, endKey, limit}})
  794. r := <-b.rangeRespc
  795. return r.keys, r.vals
  796. }
  797. func (b *fakeBatchTx) UnsafeDelete(bucketName []byte, key []byte) {
  798. b.Recorder.Record(testutil.Action{Name: "delete", Params: []interface{}{bucketName, key}})
  799. }
  800. func (b *fakeBatchTx) UnsafeForEach(bucketName []byte, visitor func(k, v []byte) error) error {
  801. return nil
  802. }
  803. func (b *fakeBatchTx) Commit() {}
  804. func (b *fakeBatchTx) CommitAndStop() {}
  805. type fakeBackend struct {
  806. tx *fakeBatchTx
  807. }
  808. func (b *fakeBackend) BatchTx() backend.BatchTx { return b.tx }
  809. func (b *fakeBackend) ReadTx() backend.ReadTx { return b.tx }
  810. func (b *fakeBackend) ConcurrentReadTx() backend.ReadTx { return b.tx }
  811. func (b *fakeBackend) Hash(ignores map[backend.IgnoreKey]struct{}) (uint32, error) { return 0, nil }
  812. func (b *fakeBackend) Size() int64 { return 0 }
  813. func (b *fakeBackend) SizeInUse() int64 { return 0 }
  814. func (b *fakeBackend) OpenReadTxN() int64 { return 0 }
  815. func (b *fakeBackend) Snapshot() backend.Snapshot { return nil }
  816. func (b *fakeBackend) ForceCommit() {}
  817. func (b *fakeBackend) Defrag() error { return nil }
  818. func (b *fakeBackend) Close() error { return nil }
  819. type indexGetResp struct {
  820. rev revision
  821. created revision
  822. ver int64
  823. err error
  824. }
  825. type indexRangeResp struct {
  826. keys [][]byte
  827. revs []revision
  828. }
  829. type indexRangeEventsResp struct {
  830. revs []revision
  831. }
  832. type fakeIndex struct {
  833. testutil.Recorder
  834. indexGetRespc chan indexGetResp
  835. indexRangeRespc chan indexRangeResp
  836. indexRangeEventsRespc chan indexRangeEventsResp
  837. indexCompactRespc chan map[revision]struct{}
  838. }
  839. func (i *fakeIndex) Revisions(key, end []byte, atRev int64) []revision {
  840. _, rev := i.Range(key, end, atRev)
  841. return rev
  842. }
  843. func (i *fakeIndex) Get(key []byte, atRev int64) (rev, created revision, ver int64, err error) {
  844. i.Recorder.Record(testutil.Action{Name: "get", Params: []interface{}{key, atRev}})
  845. r := <-i.indexGetRespc
  846. return r.rev, r.created, r.ver, r.err
  847. }
  848. func (i *fakeIndex) Range(key, end []byte, atRev int64) ([][]byte, []revision) {
  849. i.Recorder.Record(testutil.Action{Name: "range", Params: []interface{}{key, end, atRev}})
  850. r := <-i.indexRangeRespc
  851. return r.keys, r.revs
  852. }
  853. func (i *fakeIndex) Put(key []byte, rev revision) {
  854. i.Recorder.Record(testutil.Action{Name: "put", Params: []interface{}{key, rev}})
  855. }
  856. func (i *fakeIndex) Tombstone(key []byte, rev revision) error {
  857. i.Recorder.Record(testutil.Action{Name: "tombstone", Params: []interface{}{key, rev}})
  858. return nil
  859. }
  860. func (i *fakeIndex) RangeSince(key, end []byte, rev int64) []revision {
  861. i.Recorder.Record(testutil.Action{Name: "rangeEvents", Params: []interface{}{key, end, rev}})
  862. r := <-i.indexRangeEventsRespc
  863. return r.revs
  864. }
  865. func (i *fakeIndex) Compact(rev int64) map[revision]struct{} {
  866. i.Recorder.Record(testutil.Action{Name: "compact", Params: []interface{}{rev}})
  867. return <-i.indexCompactRespc
  868. }
  869. func (i *fakeIndex) Keep(rev int64) map[revision]struct{} {
  870. i.Recorder.Record(testutil.Action{Name: "keep", Params: []interface{}{rev}})
  871. return <-i.indexCompactRespc
  872. }
  873. func (i *fakeIndex) Equal(b index) bool { return false }
  874. func (i *fakeIndex) Insert(ki *keyIndex) {
  875. i.Recorder.Record(testutil.Action{Name: "insert", Params: []interface{}{ki}})
  876. }
  877. func (i *fakeIndex) KeyIndex(ki *keyIndex) *keyIndex {
  878. i.Recorder.Record(testutil.Action{Name: "keyIndex", Params: []interface{}{ki}})
  879. return nil
  880. }
  881. func createBytesSlice(bytesN, sliceN int) [][]byte {
  882. rs := [][]byte{}
  883. for len(rs) != sliceN {
  884. v := make([]byte, bytesN)
  885. if _, err := rand.Read(v); err != nil {
  886. panic(err)
  887. }
  888. rs = append(rs, v)
  889. }
  890. return rs
  891. }