Documentation ¶
Overview ¶
Package logdb implements the persistent log storage used by Dragonboat.
This package is internally used by Dragonboat, applications are not expected to import this package.
Index ¶
- Constants
- func NewDefaultBatchedLogDB(config config.NodeHostConfig, callback config.LogDBCallback, dirs []string, ...) (raftio.ILogDB, error)
- func NewDefaultLogDB(config config.NodeHostConfig, callback config.LogDBCallback, dirs []string, ...) (raftio.ILogDB, error)
- func NewLogDB(config config.NodeHostConfig, callback config.LogDBCallback, dirs []string, ...) (raftio.ILogDB, error)
- type DefaultFactory
- type IContext
- type IReusableKey
- type Key
- func (k *Key) Key() []byte
- func (k *Key) Release()
- func (k *Key) SetEntryBatchKey(clusterID uint64, nodeID uint64, batchID uint64)
- func (k *Key) SetEntryKey(clusterID uint64, nodeID uint64, index uint64)
- func (k *Key) SetMaxIndexKey(clusterID uint64, nodeID uint64)
- func (k *Key) SetMaximumKey()
- func (k *Key) SetMinimumKey()
- func (k *Key) SetStateKey(clusterID uint64, nodeID uint64)
- type LogReader
- func (lr *LogReader) Append(entries []pb.Entry) error
- func (lr *LogReader) ApplySnapshot(snapshot pb.Snapshot) error
- func (lr *LogReader) Compact(index uint64) error
- func (lr *LogReader) CreateSnapshot(snapshot pb.Snapshot) error
- func (lr *LogReader) Entries(low uint64, high uint64, maxSize uint64) ([]pb.Entry, error)
- func (lr *LogReader) GetRange() (uint64, uint64)
- func (lr *LogReader) NodeState() (pb.State, pb.Membership)
- func (lr *LogReader) SetRange(firstIndex uint64, length uint64)
- func (lr *LogReader) SetState(s pb.State)
- func (lr *LogReader) Snapshot() pb.Snapshot
- func (lr *LogReader) Term(index uint64) (uint64, error)
- type ShardedDB
- func (s *ShardedDB) BinaryFormat() uint32
- func (s *ShardedDB) Close() error
- func (s *ShardedDB) CompactEntriesTo(clusterID uint64, nodeID uint64, index uint64) (<-chan struct{}, error)
- func (s *ShardedDB) DeleteSnapshot(clusterID uint64, nodeID uint64, snapshotIndex uint64) error
- func (s *ShardedDB) GetBootstrapInfo(clusterID uint64, nodeID uint64) (pb.Bootstrap, error)
- func (s *ShardedDB) GetLogDBThreadContext() IContext
- func (s *ShardedDB) ImportSnapshot(ss pb.Snapshot, nodeID uint64) error
- func (s *ShardedDB) IterateEntries(ents []pb.Entry, size uint64, clusterID uint64, nodeID uint64, low uint64, ...) ([]pb.Entry, uint64, error)
- func (s *ShardedDB) ListNodeInfo() ([]raftio.NodeInfo, error)
- func (s *ShardedDB) ListSnapshots(clusterID uint64, nodeID uint64, index uint64) ([]pb.Snapshot, error)
- func (s *ShardedDB) Name() string
- func (s *ShardedDB) ReadRaftState(clusterID uint64, nodeID uint64, lastIndex uint64) (raftio.RaftState, error)
- func (s *ShardedDB) RemoveEntriesTo(clusterID uint64, nodeID uint64, index uint64) error
- func (s *ShardedDB) RemoveNodeData(clusterID uint64, nodeID uint64) error
- func (s *ShardedDB) SaveBootstrapInfo(clusterID uint64, nodeID uint64, bootstrap pb.Bootstrap) error
- func (s *ShardedDB) SaveRaftState(updates []pb.Update, shardID uint64) error
- func (s *ShardedDB) SaveRaftStateCtx(updates []pb.Update, ctx IContext) error
- func (s *ShardedDB) SaveSnapshots(updates []pb.Update) error
- func (s *ShardedDB) SelfCheckFailed() (bool, error)
Constants ¶
const (
// DefaultKVStoreTypeName is the type name of the default kv store
DefaultKVStoreTypeName = "rocksdb"
)
Variables ¶
This section is empty.
Functions ¶
func NewDefaultBatchedLogDB ¶
func NewDefaultBatchedLogDB(config config.NodeHostConfig, callback config.LogDBCallback, dirs []string, lldirs []string, fs vfs.IFS) (raftio.ILogDB, error)
NewDefaultBatchedLogDB creates a Log DB instance using the default KV store implementation with batched entry support.
func NewDefaultLogDB ¶
func NewDefaultLogDB(config config.NodeHostConfig, callback config.LogDBCallback, dirs []string, lldirs []string, fs vfs.IFS) (raftio.ILogDB, error)
NewDefaultLogDB creates a Log DB instance using the default KV store implementation. The created Log DB tries to store entry records in plain format but it switches to the batched mode if there is already batched entries saved in the existing DB.
func NewLogDB ¶
func NewLogDB(config config.NodeHostConfig, callback config.LogDBCallback, dirs []string, lldirs []string, batched bool, check bool, fs vfs.IFS, f kvFactory) (raftio.ILogDB, error)
NewLogDB creates a Log DB instance based on provided configuration parameters. The underlying KV store used by the Log DB instance is created by the provided factory function.
Types ¶
type DefaultFactory ¶ added in v3.3.0
type DefaultFactory struct {
// contains filtered or unexported fields
}
DefaultFactory is the default factory for creating LogDB instance.
func NewDefaultFactory ¶ added in v3.3.0
func NewDefaultFactory(fs vfs.IFS) *DefaultFactory
NewDefaultFactory creates a new DefaultFactory instance.
func (*DefaultFactory) Create ¶ added in v3.3.0
func (f *DefaultFactory) Create(cfg config.NodeHostConfig, cb config.LogDBCallback, dirs []string, lldirs []string) (raftio.ILogDB, error)
Create creates the LogDB instance.
func (*DefaultFactory) Name ¶ added in v3.3.0
func (f *DefaultFactory) Name() string
Name returns the name of the default LogDB instance.
type IContext ¶ added in v3.3.0
type IContext interface { // Destroy destroys the IContext instance. Destroy() // Reset resets the IContext instance, all previous returned keys and // buffers will be put back to the IContext instance and be ready to // be used for the next iteration. Reset() // GetKey returns a reusable key. GetKey() IReusableKey // GetValueBuffer returns a byte buffer with at least sz bytes in length. GetValueBuffer(sz uint64) []byte // GetWriteBatch returns a write batch or transaction instance. GetWriteBatch() interface{} // SetWriteBatch adds the write batch to the IContext instance. SetWriteBatch(wb interface{}) // GetEntryBatch returns an entry batch instance. GetEntryBatch() pb.EntryBatch // GetLastEntryBatch returns an entry batch instance. GetLastEntryBatch() pb.EntryBatch }
IContext is the per thread context used in the logdb module. IContext is expected to contain a list of reusable keys and byte slices that are owned per thread so they can be safely reused by the same thread when accessing ILogDB.
type IReusableKey ¶ added in v3.3.0
type IReusableKey interface { SetEntryBatchKey(clusterID uint64, nodeID uint64, index uint64) // SetEntryKey sets the key to be an entry key for the specified Raft node // with the specified entry index. SetEntryKey(clusterID uint64, nodeID uint64, index uint64) // SetStateKey sets the key to be an persistent state key suitable // for the specified Raft cluster node. SetStateKey(clusterID uint64, nodeID uint64) // SetMaxIndexKey sets the key to be the max possible index key for the // specified Raft cluster node. SetMaxIndexKey(clusterID uint64, nodeID uint64) // Key returns the underlying byte slice of the key. Key() []byte // Release releases the key instance so it can be reused in the future. Release() }
IReusableKey is the interface for keys that can be reused. A reusable key is usually obtained by calling the GetKey() function of the IContext instance.
type Key ¶ added in v3.3.0
type Key struct {
// contains filtered or unexported fields
}
Key represents keys that are managed by a sync.Pool to be reused.
func (*Key) Release ¶ added in v3.3.0
func (k *Key) Release()
Release puts the key back to the pool.
func (*Key) SetEntryBatchKey ¶ added in v3.3.0
SetEntryBatchKey sets the key value opf the entry batch.
func (*Key) SetEntryKey ¶ added in v3.3.0
SetEntryKey sets the key value to the specified entry key.
func (*Key) SetMaxIndexKey ¶ added in v3.3.0
SetMaxIndexKey sets the key value to the max index record key.
func (*Key) SetMaximumKey ¶ added in v3.3.0
func (k *Key) SetMaximumKey()
SetMaximumKey sets the key to the maximum possible value.
func (*Key) SetMinimumKey ¶ added in v3.3.0
func (k *Key) SetMinimumKey()
SetMinimumKey sets the key to the minimum possible value.
func (*Key) SetStateKey ¶ added in v3.3.0
SetStateKey sets the key value to the specified State.
type LogReader ¶
LogReader is the struct used to manage logs that have already been persisted into LogDB. This implementation is influenced by CockroachDB's replicaRaftStorage.
func NewLogReader ¶
NewLogReader creates and returns a new LogReader instance.
func (*LogReader) Append ¶
Append marks the specified entries as persisted and make them available from logreader.
func (*LogReader) ApplySnapshot ¶
ApplySnapshot applies the specified snapshot.
func (*LogReader) CreateSnapshot ¶
CreateSnapshot keeps the metadata of the specified snapshot.
func (*LogReader) Entries ¶
Entries returns persisted entries between [low, high) with a total limit of up to maxSize bytes.
func (*LogReader) GetRange ¶
GetRange returns the index range of all logs managed by the LogReader instance.
func (*LogReader) NodeState ¶
func (lr *LogReader) NodeState() (pb.State, pb.Membership)
NodeState returns the initial state.
type ShardedDB ¶ added in v3.3.0
type ShardedDB struct {
// contains filtered or unexported fields
}
ShardedDB is a LogDB implementation using sharded rocksdb instances.
func OpenShardedDB ¶ added in v3.3.0
func OpenShardedDB(config config.NodeHostConfig, cb config.LogDBCallback, dirs []string, lldirs []string, batched bool, check bool, fs vfs.IFS, kvf kvFactory) (*ShardedDB, error)
OpenShardedDB creates a ShardedDB instance.
func (*ShardedDB) BinaryFormat ¶ added in v3.3.0
BinaryFormat is the binary format supported by the sharded DB.
func (*ShardedDB) CompactEntriesTo ¶ added in v3.3.0
func (s *ShardedDB) CompactEntriesTo(clusterID uint64, nodeID uint64, index uint64) (<-chan struct{}, error)
CompactEntriesTo reclaims underlying storage space used for storing entries up to the specified index.
func (*ShardedDB) DeleteSnapshot ¶ added in v3.3.0
DeleteSnapshot removes the specified snapshot metadata from the log db.
func (*ShardedDB) GetBootstrapInfo ¶ added in v3.3.0
GetBootstrapInfo returns the saved bootstrap info for the given node.
func (*ShardedDB) GetLogDBThreadContext ¶ added in v3.3.0
GetLogDBThreadContext return an IContext instance. This method is expected to be used in benchmarks and tests only.
func (*ShardedDB) ImportSnapshot ¶ added in v3.3.0
ImportSnapshot imports the snapshot record and other metadata records to the system.
func (*ShardedDB) IterateEntries ¶ added in v3.3.0
func (s *ShardedDB) IterateEntries(ents []pb.Entry, size uint64, clusterID uint64, nodeID uint64, low uint64, high uint64, maxSize uint64) ([]pb.Entry, uint64, error)
IterateEntries returns a list of saved entries starting with index low up to index high with a max size of maxSize.
func (*ShardedDB) ListNodeInfo ¶ added in v3.3.0
ListNodeInfo lists all available NodeInfo found in the log db.
func (*ShardedDB) ListSnapshots ¶ added in v3.3.0
func (s *ShardedDB) ListSnapshots(clusterID uint64, nodeID uint64, index uint64) ([]pb.Snapshot, error)
ListSnapshots lists all available snapshots associated with the specified raft node.
func (*ShardedDB) ReadRaftState ¶ added in v3.3.0
func (s *ShardedDB) ReadRaftState(clusterID uint64, nodeID uint64, lastIndex uint64) (raftio.RaftState, error)
ReadRaftState returns the persistent state of the specified raft node.
func (*ShardedDB) RemoveEntriesTo ¶ added in v3.3.0
RemoveEntriesTo removes entries associated with the specified raft node up to the specified index.
func (*ShardedDB) RemoveNodeData ¶ added in v3.3.0
RemoveNodeData deletes all node data that belongs to the specified node.
func (*ShardedDB) SaveBootstrapInfo ¶ added in v3.3.0
func (s *ShardedDB) SaveBootstrapInfo(clusterID uint64, nodeID uint64, bootstrap pb.Bootstrap) error
SaveBootstrapInfo saves the specified bootstrap info for the given node.
func (*ShardedDB) SaveRaftState ¶ added in v3.3.0
SaveRaftState saves the raft state and logs found in the raft.Update list to the log db.
func (*ShardedDB) SaveRaftStateCtx ¶ added in v3.3.0
SaveRaftStateCtx saves the raft state and logs found in the raft.Update list to the log db.
func (*ShardedDB) SaveSnapshots ¶ added in v3.3.0
SaveSnapshots saves all snapshot metadata found in the raft.Update list.
func (*ShardedDB) SelfCheckFailed ¶ added in v3.3.0
SelfCheckFailed runs a self check on all db shards and report whether any failure is observed.