cosmos-sdk/store/iavl/store.go

493 lines
12 KiB
Go
Raw Normal View History

2019-02-01 17:03:09 -08:00
package iavl
2017-11-29 03:08:32 -08:00
import (
"fmt"
"io"
2017-12-10 00:24:55 -08:00
"sync"
"time"
2017-11-29 04:19:41 -08:00
ics23iavl "github.com/confio/ics23-iavl"
ics23 "github.com/confio/ics23/go"
2017-11-29 03:08:32 -08:00
"github.com/tendermint/iavl"
abci "github.com/tendermint/tendermint/abci/types"
"github.com/tendermint/tendermint/crypto/merkle"
dbm "github.com/tendermint/tm-db"
"github.com/cosmos/cosmos-sdk/store/cachekv"
"github.com/cosmos/cosmos-sdk/store/tracekv"
"github.com/cosmos/cosmos-sdk/store/types"
2020-06-18 11:12:44 -07:00
"github.com/cosmos/cosmos-sdk/telemetry"
sdk "github.com/cosmos/cosmos-sdk/types"
sdkerrors "github.com/cosmos/cosmos-sdk/types/errors"
"github.com/cosmos/cosmos-sdk/types/kv"
)
const (
2018-07-12 18:20:26 -07:00
defaultIAVLCacheSize = 10000
)
2017-11-29 04:19:41 -08:00
var (
_ types.KVStore = (*Store)(nil)
_ types.CommitStore = (*Store)(nil)
_ types.CommitKVStore = (*Store)(nil)
_ types.Queryable = (*Store)(nil)
)
// Store Implements types.KVStore and CommitKVStore.
type Store struct {
2020-06-22 13:31:33 -07:00
tree Tree
}
// LoadStore returns an IAVL Store as a CommitKVStore. Internally, it will load the
// store's version (id) from the provided DB. An error is returned if the version
// fails to load.
2020-06-22 13:31:33 -07:00
func LoadStore(db dbm.DB, id types.CommitID, lazyLoading bool) (types.CommitKVStore, error) {
tree, err := iavl.NewMutableTree(db, defaultIAVLCacheSize)
if err != nil {
return nil, err
}
if lazyLoading {
_, err = tree.LazyLoadVersion(id.Version)
} else {
_, err = tree.LoadVersion(id.Version)
}
if err != nil {
return nil, err
}
return &Store{
2020-06-22 13:31:33 -07:00
tree: tree,
}, nil
}
2020-01-22 11:52:56 -08:00
// UnsafeNewStore returns a reference to a new IAVL Store with a given mutable
// IAVL tree reference. It should only be used for testing purposes.
//
// CONTRACT: The IAVL tree should be fully loaded.
// CONTRACT: PruningOptions passed in as argument must be the same as pruning options
// passed into iavl.MutableTree
2020-06-22 13:31:33 -07:00
func UnsafeNewStore(tree *iavl.MutableTree) *Store {
return &Store{
2020-06-22 13:31:33 -07:00
tree: tree,
}
2017-11-29 05:07:39 -08:00
}
// GetImmutable returns a reference to a new store backed by an immutable IAVL
// tree at a specific version (height) without any pruning options. This should
// be used for querying and iteration only. If the version does not exist or has
// been pruned, an error will be returned. Any mutable operations executed will
// result in a panic.
func (st *Store) GetImmutable(version int64) (*Store, error) {
if !st.VersionExists(version) {
return nil, iavl.ErrVersionDoesNotExist
}
iTree, err := st.tree.GetImmutable(version)
if err != nil {
return nil, err
}
return &Store{
2020-06-22 13:31:33 -07:00
tree: &immutableTree{iTree},
}, nil
}
// Commit commits the current store state and returns a CommitID with the new
// version and hash.
2019-02-01 17:03:09 -08:00
func (st *Store) Commit() types.CommitID {
defer telemetry.MeasureSince(time.Now(), "store", "iavl", "commit")
2020-06-18 11:12:44 -07:00
2017-12-09 12:35:51 -08:00
hash, version, err := st.tree.SaveVersion()
2017-11-29 04:19:41 -08:00
if err != nil {
panic(err)
}
2019-02-01 17:03:09 -08:00
return types.CommitID{
2017-12-09 12:35:51 -08:00
Version: version,
2017-11-29 04:19:41 -08:00
Hash: hash,
}
2017-11-29 03:08:32 -08:00
}
// Implements Committer.
2019-02-01 17:03:09 -08:00
func (st *Store) LastCommitID() types.CommitID {
return types.CommitID{
2018-09-09 08:13:17 -07:00
Version: st.tree.Version(),
Hash: st.tree.Hash(),
}
}
2020-01-22 11:52:56 -08:00
// SetPruning panics as pruning options should be provided at initialization
// since IAVl accepts pruning options directly.
func (st *Store) SetPruning(_ types.PruningOptions) {
panic("cannot set pruning options on an initialized IAVL store")
2018-07-12 18:20:26 -07:00
}
// VersionExists returns whether or not a given version is stored.
2019-02-01 17:03:09 -08:00
func (st *Store) VersionExists(version int64) bool {
2018-06-26 16:26:24 -07:00
return st.tree.VersionExists(version)
}
// Implements Store.
2019-02-01 17:03:09 -08:00
func (st *Store) GetStoreType() types.StoreType {
return types.StoreTypeIAVL
}
// Implements Store.
2019-02-01 17:03:09 -08:00
func (st *Store) CacheWrap() types.CacheWrap {
return cachekv.NewStore(st)
2017-12-10 00:24:55 -08:00
}
// CacheWrapWithTrace implements the Store interface.
2019-02-01 17:03:09 -08:00
func (st *Store) CacheWrapWithTrace(w io.Writer, tc types.TraceContext) types.CacheWrap {
return cachekv.NewStore(tracekv.NewStore(st, w, tc))
}
2019-02-01 17:03:09 -08:00
// Implements types.KVStore.
func (st *Store) Set(key, value []byte) {
defer telemetry.MeasureSince(time.Now(), "store", "iavl", "set")
types.AssertValidKey(key)
2019-02-05 10:39:22 -08:00
types.AssertValidValue(value)
2017-12-09 12:35:51 -08:00
st.tree.Set(key, value)
2017-11-29 05:07:39 -08:00
}
2019-02-01 17:03:09 -08:00
// Implements types.KVStore.
func (st *Store) Get(key []byte) []byte {
defer telemetry.MeasureSince(time.Now(), "store", "iavl", "get")
_, value := st.tree.Get(key)
return value
2017-11-29 05:07:39 -08:00
}
2019-02-01 17:03:09 -08:00
// Implements types.KVStore.
func (st *Store) Has(key []byte) (exists bool) {
defer telemetry.MeasureSince(time.Now(), "store", "iavl", "has")
2017-12-09 12:35:51 -08:00
return st.tree.Has(key)
2017-11-29 05:07:39 -08:00
}
2019-02-01 17:03:09 -08:00
// Implements types.KVStore.
func (st *Store) Delete(key []byte) {
defer telemetry.MeasureSince(time.Now(), "store", "iavl", "delete")
2017-12-11 23:30:44 -08:00
st.tree.Remove(key)
2017-11-29 05:07:39 -08:00
}
2020-06-22 13:31:33 -07:00
// DeleteVersions deletes a series of versions from the MutableTree. An error
// is returned if any single version is invalid or the delete fails. All writes
// happen in a single batch with a single commit.
func (st *Store) DeleteVersions(versions ...int64) error {
return st.tree.DeleteVersions(versions...)
}
2019-02-01 17:03:09 -08:00
// Implements types.KVStore.
func (st *Store) Iterator(start, end []byte) types.Iterator {
var iTree *iavl.ImmutableTree
switch tree := st.tree.(type) {
case *immutableTree:
iTree = tree.ImmutableTree
case *iavl.MutableTree:
iTree = tree.ImmutableTree
}
return newIAVLIterator(iTree, start, end, true)
}
2019-02-01 17:03:09 -08:00
// Implements types.KVStore.
func (st *Store) ReverseIterator(start, end []byte) types.Iterator {
var iTree *iavl.ImmutableTree
switch tree := st.tree.(type) {
case *immutableTree:
iTree = tree.ImmutableTree
case *iavl.MutableTree:
iTree = tree.ImmutableTree
}
return newIAVLIterator(iTree, start, end, false)
2018-04-01 09:00:28 -07:00
}
// Handle gatest the latest height, if height is 0
func getHeight(tree Tree, req abci.RequestQuery) int64 {
height := req.Height
if height == 0 {
2018-09-09 08:13:17 -07:00
latest := tree.Version()
if tree.VersionExists(latest - 1) {
height = latest - 1
} else {
height = latest
}
}
return height
}
2018-01-30 10:37:03 -08:00
// Query implements ABCI interface, allows queries
//
// by default we will return from (latest height -1),
// as we will have merkle proofs immediately (header height = data height + 1)
// If latest-1 is not present, use latest (which must be present)
// if you care to have the latest data to see a tx results, you must
// explicitly set the height you want to see
2019-02-01 17:03:09 -08:00
func (st *Store) Query(req abci.RequestQuery) (res abci.ResponseQuery) {
defer telemetry.MeasureSince(time.Now(), "store", "iavl", "query")
2020-06-18 11:12:44 -07:00
2018-01-30 10:37:03 -08:00
if len(req.Data) == 0 {
return sdkerrors.QueryResult(sdkerrors.Wrap(sdkerrors.ErrTxDecode, "query cannot be zero length"))
2018-01-30 10:37:03 -08:00
}
tree := st.tree
// store the height we chose in the response, with 0 being changed to the
// latest height
res.Height = getHeight(tree, req)
2018-01-30 10:37:03 -08:00
switch req.Path {
case "/key": // get by key
key := req.Data // data holds the key bytes
2018-01-30 10:37:03 -08:00
res.Key = key
2018-07-12 18:20:26 -07:00
if !st.VersionExists(res.Height) {
res.Log = iavl.ErrVersionDoesNotExist.Error()
2018-07-12 18:20:26 -07:00
break
}
_, res.Value = tree.GetVersioned(key, res.Height)
if !req.Prove {
break
2018-01-30 10:37:03 -08:00
}
// Continue to prove existence/absence of value
// Must convert store.Tree to iavl.MutableTree with given version to use in CreateProof
iTree, err := tree.GetImmutable(res.Height)
if err != nil {
// sanity check: If value for given version was retrieved, immutable tree must also be retrievable
panic(fmt.Sprintf("version exists in store but could not retrieve corresponding versioned tree in store, %s", err.Error()))
}
mtree := &iavl.MutableTree{
ImmutableTree: iTree,
}
// get proof from tree and convert to merkle.Proof before adding to result
res.Proof = getProofFromTree(mtree, req.Data, res.Value != nil)
case "/subspace":
2019-02-01 17:03:09 -08:00
var KVs []types.KVPair
subspace := req.Data
res.Key = subspace
2019-02-05 10:39:22 -08:00
iterator := types.KVStorePrefixIterator(st, subspace)
for ; iterator.Valid(); iterator.Next() {
2019-02-01 17:03:09 -08:00
KVs = append(KVs, types.KVPair{Key: iterator.Key(), Value: iterator.Value()})
}
iterator.Close()
res.Value = cdc.MustMarshalBinaryBare(KVs)
2018-01-30 10:37:03 -08:00
default:
return sdkerrors.QueryResult(sdkerrors.Wrapf(sdkerrors.ErrUnknownRequest, "unexpected query path: %v", req.Path))
2018-01-30 10:37:03 -08:00
}
2019-09-19 06:21:38 -07:00
return res
2018-01-30 10:37:03 -08:00
}
// Takes a MutableTree, a key, and a flag for creating existence or absence proof and returns the
// appropriate merkle.Proof. Since this must be called after querying for the value, this function should never error
// Thus, it will panic on error rather than returning it
func getProofFromTree(tree *iavl.MutableTree, key []byte, exists bool) *merkle.Proof {
var (
commitmentProof *ics23.CommitmentProof
err error
)
if exists {
// value was found
commitmentProof, err = ics23iavl.CreateMembershipProof(tree, key)
if err != nil {
// sanity check: If value was found, membership proof must be creatable
panic(fmt.Sprintf("unexpected value for empty proof: %s", err.Error()))
}
} else {
// value wasn't found
commitmentProof, err = ics23iavl.CreateNonMembershipProof(tree, key)
if err != nil {
// sanity check: If value wasn't found, nonmembership proof must be creatable
panic(fmt.Sprintf("unexpected error for nonexistence proof: %s", err.Error()))
}
}
2020-06-22 13:31:33 -07:00
op := types.NewIavlCommitmentOp(key, commitmentProof)
return &merkle.Proof{Ops: []merkle.ProofOp{op.ProofOp()}}
}
2017-12-09 12:35:51 -08:00
//----------------------------------------
2019-02-01 17:03:09 -08:00
// Implements types.Iterator.
type iavlIterator struct {
2017-12-10 00:24:55 -08:00
// Domain
start, end []byte
key []byte // The current key (mutable)
value []byte // The current value (mutable)
// Underlying store
tree *iavl.ImmutableTree
2017-12-10 00:24:55 -08:00
// Channel to push iteration values.
iterCh chan kv.Pair
2017-12-10 00:24:55 -08:00
// Close this to release goroutine.
quitCh chan struct{}
// Close this to signal that state is initialized.
initCh chan struct{}
mtx sync.Mutex
ascending bool // Iteration order
invalid bool // True once, true forever (mutable)
}
2019-02-01 17:03:09 -08:00
var _ types.Iterator = (*iavlIterator)(nil)
2017-12-10 00:24:55 -08:00
// newIAVLIterator will create a new iavlIterator.
// CONTRACT: Caller must release the iavlIterator, as each one creates a new
// goroutine.
2018-08-03 16:24:03 -07:00
func newIAVLIterator(tree *iavl.ImmutableTree, start, end []byte, ascending bool) *iavlIterator {
2017-12-11 23:30:44 -08:00
iter := &iavlIterator{
tree: tree,
start: sdk.CopyBytes(start),
end: sdk.CopyBytes(end),
2017-12-10 00:24:55 -08:00
ascending: ascending,
iterCh: make(chan kv.Pair), // Set capacity > 0?
2017-12-10 00:24:55 -08:00
quitCh: make(chan struct{}),
initCh: make(chan struct{}),
}
2017-12-11 23:30:44 -08:00
go iter.iterateRoutine()
go iter.initRoutine()
return iter
2017-12-10 00:24:55 -08:00
}
// Run this to funnel items from the tree to iterCh.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) iterateRoutine() {
iter.tree.IterateRange(
iter.start, iter.end, iter.ascending,
2017-12-10 00:24:55 -08:00
func(key, value []byte) bool {
select {
2017-12-11 23:30:44 -08:00
case <-iter.quitCh:
2017-12-10 00:24:55 -08:00
return true // done with iteration.
case iter.iterCh <- kv.Pair{Key: key, Value: value}:
2017-12-10 00:24:55 -08:00
return false // yay.
}
},
)
2017-12-11 23:30:44 -08:00
close(iter.iterCh) // done.
2017-12-10 00:24:55 -08:00
}
// Run this to fetch the first item.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) initRoutine() {
iter.receiveNext()
close(iter.initCh)
2017-12-10 00:24:55 -08:00
}
2019-02-01 17:03:09 -08:00
// Implements types.Iterator.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) Domain() (start, end []byte) {
return iter.start, iter.end
}
2019-02-01 17:03:09 -08:00
// Implements types.Iterator.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) Valid() bool {
iter.waitInit()
iter.mtx.Lock()
2017-12-10 00:24:55 -08:00
validity := !iter.invalid
iter.mtx.Unlock()
return validity
}
2019-02-01 17:03:09 -08:00
// Implements types.Iterator.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) Next() {
iter.waitInit()
iter.mtx.Lock()
iter.assertIsValid(true)
2017-12-10 00:24:55 -08:00
2017-12-11 23:30:44 -08:00
iter.receiveNext()
iter.mtx.Unlock()
}
2019-02-01 17:03:09 -08:00
// Implements types.Iterator.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) Key() []byte {
iter.waitInit()
iter.mtx.Lock()
iter.assertIsValid(true)
2017-12-10 00:24:55 -08:00
key := iter.key
iter.mtx.Unlock()
return key
}
2019-02-01 17:03:09 -08:00
// Implements types.Iterator.
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) Value() []byte {
iter.waitInit()
iter.mtx.Lock()
iter.assertIsValid(true)
2017-12-10 00:24:55 -08:00
val := iter.value
iter.mtx.Unlock()
return val
}
// Close closes the IAVL iterator by closing the quit channel and waiting for
// the iterCh to finish/close.
func (iter *iavlIterator) Close() {
2017-12-11 23:30:44 -08:00
close(iter.quitCh)
// wait iterCh to close
for range iter.iterCh {
}
2017-12-10 00:24:55 -08:00
}
// Error performs a no-op.
func (iter *iavlIterator) Error() error {
return nil
}
2017-12-10 00:24:55 -08:00
//----------------------------------------
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) setNext(key, value []byte) {
iter.assertIsValid(false)
2017-12-10 00:24:55 -08:00
2017-12-11 23:30:44 -08:00
iter.key = key
iter.value = value
2017-12-10 00:24:55 -08:00
}
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) setInvalid() {
iter.assertIsValid(false)
2017-12-10 00:24:55 -08:00
2017-12-11 23:30:44 -08:00
iter.invalid = true
2017-12-10 00:24:55 -08:00
}
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) waitInit() {
<-iter.initCh
2017-12-10 00:24:55 -08:00
}
2017-12-11 23:30:44 -08:00
func (iter *iavlIterator) receiveNext() {
kvPair, ok := <-iter.iterCh
2017-12-10 00:24:55 -08:00
if ok {
2017-12-11 23:30:44 -08:00
iter.setNext(kvPair.Key, kvPair.Value)
2017-12-10 00:24:55 -08:00
} else {
2017-12-11 23:30:44 -08:00
iter.setInvalid()
2017-12-10 00:24:55 -08:00
}
}
// assertIsValid panics if the iterator is invalid. If unlockMutex is true,
// it also unlocks the mutex before panicing, to prevent deadlocks in code that
// recovers from panics
func (iter *iavlIterator) assertIsValid(unlockMutex bool) {
2017-12-11 23:30:44 -08:00
if iter.invalid {
if unlockMutex {
iter.mtx.Unlock()
}
2017-12-10 00:24:55 -08:00
panic("invalid iterator")
}
}