You can not select more than 25 topics Topics must start with a letter or number, can include dashes ('-') and can be up to 35 characters long.

491 lines
12 KiB

  1. package kv
  2. import (
  3. "context"
  4. "errors"
  5. "fmt"
  6. "sort"
  7. "strconv"
  8. "strings"
  9. "github.com/google/orderedcode"
  10. dbm "github.com/tendermint/tm-db"
  11. abci "github.com/tendermint/tendermint/abci/types"
  12. "github.com/tendermint/tendermint/internal/pubsub/query"
  13. "github.com/tendermint/tendermint/internal/pubsub/query/syntax"
  14. "github.com/tendermint/tendermint/internal/state/indexer"
  15. "github.com/tendermint/tendermint/types"
  16. )
  17. var _ indexer.BlockIndexer = (*BlockerIndexer)(nil)
  18. // BlockerIndexer implements a block indexer, indexing BeginBlock and EndBlock
  19. // events with an underlying KV store. Block events are indexed by their height,
  20. // such that matching search criteria returns the respective block height(s).
  21. type BlockerIndexer struct {
  22. store dbm.DB
  23. }
  24. func New(store dbm.DB) *BlockerIndexer {
  25. return &BlockerIndexer{
  26. store: store,
  27. }
  28. }
  29. // Has returns true if the given height has been indexed. An error is returned
  30. // upon database query failure.
  31. func (idx *BlockerIndexer) Has(height int64) (bool, error) {
  32. key, err := heightKey(height)
  33. if err != nil {
  34. return false, fmt.Errorf("failed to create block height index key: %w", err)
  35. }
  36. return idx.store.Has(key)
  37. }
  38. // Index indexes BeginBlock and EndBlock events for a given block by its height.
  39. // The following is indexed:
  40. //
  41. // primary key: encode(block.height | height) => encode(height)
  42. // BeginBlock events: encode(eventType.eventAttr|eventValue|height|begin_block) => encode(height)
  43. // EndBlock events: encode(eventType.eventAttr|eventValue|height|end_block) => encode(height)
  44. func (idx *BlockerIndexer) Index(bh types.EventDataNewBlockHeader) error {
  45. batch := idx.store.NewBatch()
  46. defer batch.Close()
  47. height := bh.Header.Height
  48. // 1. index by height
  49. key, err := heightKey(height)
  50. if err != nil {
  51. return fmt.Errorf("failed to create block height index key: %w", err)
  52. }
  53. if err := batch.Set(key, int64ToBytes(height)); err != nil {
  54. return err
  55. }
  56. // 2. index BeginBlock events
  57. if err := idx.indexEvents(batch, bh.ResultBeginBlock.Events, "begin_block", height); err != nil {
  58. return fmt.Errorf("failed to index BeginBlock events: %w", err)
  59. }
  60. // 3. index EndBlock events
  61. if err := idx.indexEvents(batch, bh.ResultEndBlock.Events, "end_block", height); err != nil {
  62. return fmt.Errorf("failed to index EndBlock events: %w", err)
  63. }
  64. return batch.WriteSync()
  65. }
  66. // Search performs a query for block heights that match a given BeginBlock
  67. // and Endblock event search criteria. The given query can match against zero,
  68. // one or more block heights. In the case of height queries, i.e. block.height=H,
  69. // if the height is indexed, that height alone will be returned. An error and
  70. // nil slice is returned. Otherwise, a non-nil slice and nil error is returned.
  71. func (idx *BlockerIndexer) Search(ctx context.Context, q *query.Query) ([]int64, error) {
  72. results := make([]int64, 0)
  73. select {
  74. case <-ctx.Done():
  75. return results, nil
  76. default:
  77. }
  78. conditions := q.Syntax()
  79. // If there is an exact height query, return the result immediately
  80. // (if it exists).
  81. height, ok := lookForHeight(conditions)
  82. if ok {
  83. ok, err := idx.Has(height)
  84. if err != nil {
  85. return nil, err
  86. }
  87. if ok {
  88. return []int64{height}, nil
  89. }
  90. return results, nil
  91. }
  92. var heightsInitialized bool
  93. filteredHeights := make(map[string][]byte)
  94. // conditions to skip because they're handled before "everything else"
  95. skipIndexes := make([]int, 0)
  96. // Extract ranges. If both upper and lower bounds exist, it's better to get
  97. // them in order as to not iterate over kvs that are not within range.
  98. ranges, rangeIndexes := indexer.LookForRanges(conditions)
  99. if len(ranges) > 0 {
  100. skipIndexes = append(skipIndexes, rangeIndexes...)
  101. for _, qr := range ranges {
  102. prefix, err := orderedcode.Append(nil, qr.Key)
  103. if err != nil {
  104. return nil, fmt.Errorf("failed to create prefix key: %w", err)
  105. }
  106. if !heightsInitialized {
  107. filteredHeights, err = idx.matchRange(ctx, qr, prefix, filteredHeights, true)
  108. if err != nil {
  109. return nil, err
  110. }
  111. heightsInitialized = true
  112. // Ignore any remaining conditions if the first condition resulted in no
  113. // matches (assuming implicit AND operand).
  114. if len(filteredHeights) == 0 {
  115. break
  116. }
  117. } else {
  118. filteredHeights, err = idx.matchRange(ctx, qr, prefix, filteredHeights, false)
  119. if err != nil {
  120. return nil, err
  121. }
  122. }
  123. }
  124. }
  125. // for all other conditions
  126. for i, c := range conditions {
  127. if intInSlice(i, skipIndexes) {
  128. continue
  129. }
  130. startKey, err := orderedcode.Append(nil, c.Tag, c.Arg.Value())
  131. if err != nil {
  132. return nil, err
  133. }
  134. if !heightsInitialized {
  135. filteredHeights, err = idx.match(ctx, c, startKey, filteredHeights, true)
  136. if err != nil {
  137. return nil, err
  138. }
  139. heightsInitialized = true
  140. // Ignore any remaining conditions if the first condition resulted in no
  141. // matches (assuming implicit AND operand).
  142. if len(filteredHeights) == 0 {
  143. break
  144. }
  145. } else {
  146. filteredHeights, err = idx.match(ctx, c, startKey, filteredHeights, false)
  147. if err != nil {
  148. return nil, err
  149. }
  150. }
  151. }
  152. // fetch matching heights
  153. results = make([]int64, 0, len(filteredHeights))
  154. heights:
  155. for _, hBz := range filteredHeights {
  156. h := int64FromBytes(hBz)
  157. ok, err := idx.Has(h)
  158. if err != nil {
  159. return nil, err
  160. }
  161. if ok {
  162. results = append(results, h)
  163. }
  164. select {
  165. case <-ctx.Done():
  166. break heights
  167. default:
  168. }
  169. }
  170. sort.Slice(results, func(i, j int) bool { return results[i] < results[j] })
  171. return results, nil
  172. }
  173. // matchRange returns all matching block heights that match a given QueryRange
  174. // and start key. An already filtered result (filteredHeights) is provided such
  175. // that any non-intersecting matches are removed.
  176. //
  177. // NOTE: The provided filteredHeights may be empty if no previous condition has
  178. // matched.
  179. func (idx *BlockerIndexer) matchRange(
  180. ctx context.Context,
  181. qr indexer.QueryRange,
  182. startKey []byte,
  183. filteredHeights map[string][]byte,
  184. firstRun bool,
  185. ) (map[string][]byte, error) {
  186. // A previous match was attempted but resulted in no matches, so we return
  187. // no matches (assuming AND operand).
  188. if !firstRun && len(filteredHeights) == 0 {
  189. return filteredHeights, nil
  190. }
  191. tmpHeights := make(map[string][]byte)
  192. lowerBound := qr.LowerBoundValue()
  193. upperBound := qr.UpperBoundValue()
  194. it, err := dbm.IteratePrefix(idx.store, startKey)
  195. if err != nil {
  196. return nil, fmt.Errorf("failed to create prefix iterator: %w", err)
  197. }
  198. defer it.Close()
  199. iter:
  200. for ; it.Valid(); it.Next() {
  201. var (
  202. eventValue string
  203. err error
  204. )
  205. if qr.Key == types.BlockHeightKey {
  206. eventValue, err = parseValueFromPrimaryKey(it.Key())
  207. } else {
  208. eventValue, err = parseValueFromEventKey(it.Key())
  209. }
  210. if err != nil {
  211. continue
  212. }
  213. if _, ok := qr.AnyBound().(int64); ok {
  214. v, err := strconv.ParseInt(eventValue, 10, 64)
  215. if err != nil {
  216. continue iter
  217. }
  218. include := true
  219. if lowerBound != nil && v < lowerBound.(int64) {
  220. include = false
  221. }
  222. if upperBound != nil && v > upperBound.(int64) {
  223. include = false
  224. }
  225. if include {
  226. tmpHeights[string(it.Value())] = it.Value()
  227. }
  228. }
  229. select {
  230. case <-ctx.Done():
  231. break iter
  232. default:
  233. }
  234. }
  235. if err := it.Error(); err != nil {
  236. return nil, err
  237. }
  238. if len(tmpHeights) == 0 || firstRun {
  239. // Either:
  240. //
  241. // 1. Regardless if a previous match was attempted, which may have had
  242. // results, but no match was found for the current condition, then we
  243. // return no matches (assuming AND operand).
  244. //
  245. // 2. A previous match was not attempted, so we return all results.
  246. return tmpHeights, nil
  247. }
  248. // Remove/reduce matches in filteredHashes that were not found in this
  249. // match (tmpHashes).
  250. for k := range filteredHeights {
  251. if tmpHeights[k] == nil {
  252. delete(filteredHeights, k)
  253. select {
  254. case <-ctx.Done():
  255. break
  256. default:
  257. }
  258. }
  259. }
  260. return filteredHeights, nil
  261. }
  262. // match returns all matching heights that meet a given query condition and start
  263. // key. An already filtered result (filteredHeights) is provided such that any
  264. // non-intersecting matches are removed.
  265. //
  266. // NOTE: The provided filteredHeights may be empty if no previous condition has
  267. // matched.
  268. func (idx *BlockerIndexer) match(
  269. ctx context.Context,
  270. c syntax.Condition,
  271. startKeyBz []byte,
  272. filteredHeights map[string][]byte,
  273. firstRun bool,
  274. ) (map[string][]byte, error) {
  275. // A previous match was attempted but resulted in no matches, so we return
  276. // no matches (assuming AND operand).
  277. if !firstRun && len(filteredHeights) == 0 {
  278. return filteredHeights, nil
  279. }
  280. tmpHeights := make(map[string][]byte)
  281. switch {
  282. case c.Op == syntax.TEq:
  283. it, err := dbm.IteratePrefix(idx.store, startKeyBz)
  284. if err != nil {
  285. return nil, fmt.Errorf("failed to create prefix iterator: %w", err)
  286. }
  287. defer it.Close()
  288. for ; it.Valid(); it.Next() {
  289. tmpHeights[string(it.Value())] = it.Value()
  290. if err := ctx.Err(); err != nil {
  291. break
  292. }
  293. }
  294. if err := it.Error(); err != nil {
  295. return nil, err
  296. }
  297. case c.Op == syntax.TExists:
  298. prefix, err := orderedcode.Append(nil, c.Tag)
  299. if err != nil {
  300. return nil, err
  301. }
  302. it, err := dbm.IteratePrefix(idx.store, prefix)
  303. if err != nil {
  304. return nil, fmt.Errorf("failed to create prefix iterator: %w", err)
  305. }
  306. defer it.Close()
  307. iterExists:
  308. for ; it.Valid(); it.Next() {
  309. tmpHeights[string(it.Value())] = it.Value()
  310. select {
  311. case <-ctx.Done():
  312. break iterExists
  313. default:
  314. }
  315. }
  316. if err := it.Error(); err != nil {
  317. return nil, err
  318. }
  319. case c.Op == syntax.TContains:
  320. prefix, err := orderedcode.Append(nil, c.Tag)
  321. if err != nil {
  322. return nil, err
  323. }
  324. it, err := dbm.IteratePrefix(idx.store, prefix)
  325. if err != nil {
  326. return nil, fmt.Errorf("failed to create prefix iterator: %w", err)
  327. }
  328. defer it.Close()
  329. iterContains:
  330. for ; it.Valid(); it.Next() {
  331. eventValue, err := parseValueFromEventKey(it.Key())
  332. if err != nil {
  333. continue
  334. }
  335. if strings.Contains(eventValue, c.Arg.Value()) {
  336. tmpHeights[string(it.Value())] = it.Value()
  337. }
  338. select {
  339. case <-ctx.Done():
  340. break iterContains
  341. default:
  342. }
  343. }
  344. if err := it.Error(); err != nil {
  345. return nil, err
  346. }
  347. default:
  348. return nil, errors.New("other operators should be handled already")
  349. }
  350. if len(tmpHeights) == 0 || firstRun {
  351. // Either:
  352. //
  353. // 1. Regardless if a previous match was attempted, which may have had
  354. // results, but no match was found for the current condition, then we
  355. // return no matches (assuming AND operand).
  356. //
  357. // 2. A previous match was not attempted, so we return all results.
  358. return tmpHeights, nil
  359. }
  360. // Remove/reduce matches in filteredHeights that were not found in this
  361. // match (tmpHeights).
  362. for k := range filteredHeights {
  363. if tmpHeights[k] == nil {
  364. delete(filteredHeights, k)
  365. select {
  366. case <-ctx.Done():
  367. break
  368. default:
  369. }
  370. }
  371. }
  372. return filteredHeights, nil
  373. }
  374. func (idx *BlockerIndexer) indexEvents(batch dbm.Batch, events []abci.Event, typ string, height int64) error {
  375. heightBz := int64ToBytes(height)
  376. for _, event := range events {
  377. // only index events with a non-empty type
  378. if len(event.Type) == 0 {
  379. continue
  380. }
  381. for _, attr := range event.Attributes {
  382. if len(attr.Key) == 0 {
  383. continue
  384. }
  385. // index iff the event specified index:true and it's not a reserved event
  386. compositeKey := fmt.Sprintf("%s.%s", event.Type, attr.Key)
  387. if compositeKey == types.BlockHeightKey {
  388. return fmt.Errorf("event type and attribute key \"%s\" is reserved; please use a different key", compositeKey)
  389. }
  390. if attr.GetIndex() {
  391. key, err := eventKey(compositeKey, typ, attr.Value, height)
  392. if err != nil {
  393. return fmt.Errorf("failed to create block index key: %w", err)
  394. }
  395. if err := batch.Set(key, heightBz); err != nil {
  396. return err
  397. }
  398. }
  399. }
  400. }
  401. return nil
  402. }