keyspan

package
v0.0.0-...-417737b Latest Latest
Warning

This package is not in the latest version of its module.

Go to latest
Published: Jun 21, 2022 License: BSD-3-Clause Imports: 9 Imported by: 0

Documentation

Overview

Package keyspan provides facilities for sorting, fragmenting and iterating over spans of user keys.

A Span represents a range of user key space with an inclusive start key and exclusive end key. A span may hold any number of Keys which are applied over the entirety of the span's keyspace.

Spans are used within Pebble as an in-memory representation of range deletion tombstones, and range key sets, unsets and deletes. Spans are fragmented at overlapping key boundaries by the Fragmenter type. This package's various iteration facilities require these non-overlapping fragmented spans.

Index

Constants

This section is empty.

Variables

This section is empty.

Functions

func Sort

func Sort(cmp base.Compare, spans []Span)

Sort the spans by start key. This is the ordering required by the Fragmenter. Usually spans are naturally sorted by their start key, but that isn't true for range deletion tombstones in the legacy range-del-v1 block format.

func SortKeys

func SortKeys(keys []Key)

SortKeys sorts a keys slice by trailer.

Types

type DefragmentMethod

type DefragmentMethod interface {
	// ShouldDefragment takes two abutting spans and returns whether the two
	// spans should be combined into a single, defragmented Span.
	ShouldDefragment(cmp base.Compare, left, right *Span) bool
}

DefragmentMethod configures the defragmentation performed by the DefragmentingIter.

var DefragmentInternal DefragmentMethod = DefragmentMethodFunc(func(cmp base.Compare, a, b *Span) bool {
	if len(a.Keys) != len(b.Keys) {
		return false
	}
	for i := range a.Keys {
		if a.Keys[i].Trailer != b.Keys[i].Trailer {
			return false
		}
		if cmp(a.Keys[i].Suffix, b.Keys[i].Suffix) != 0 {
			return false
		}
		if !bytes.Equal(a.Keys[i].Value, b.Keys[i].Value) {
			return false
		}
	}
	return true
})

DefragmentInternal configures a DefragmentingIter to defragment spans only if they have identical keys.

This defragmenting method is intended for use in compactions that may see internal range keys fragments that may now be joined, because the state that required their fragmentation has been dropped.

type DefragmentMethodFunc

type DefragmentMethodFunc func(cmp base.Compare, left, right *Span) bool

The DefragmentMethodFunc type is an adapter to allow the use of ordinary functions as DefragmentMethods. If f is a function with the appropriate signature, DefragmentMethodFunc(f) is a DefragmentMethod that calls f.

func (DefragmentMethodFunc) ShouldDefragment

func (f DefragmentMethodFunc) ShouldDefragment(cmp base.Compare, left, right *Span) bool

ShouldDefragment calls f(cmp, left, right).

type DefragmentReducer

type DefragmentReducer func(cur, next []Key) []Key

DefragmentReducer merges the current and next Key slices, returning a new Key slice.

Implementations should modify and return `cur` to save on allocations, or consider allocating a new slice, as the `cur` slice may be retained by the DefragmentingIter and mutated. The `next` slice must not be mutated.

The incoming slices are sorted by (SeqNum, Kind) descending. The output slice must also have this sort order.

var StaticDefragmentReducer DefragmentReducer = func(cur, _ []Key) []Key {
	return cur
}

StaticDefragmentReducer is a no-op DefragmentReducer that simply returns the current key slice, effectively retaining the first set of keys encountered for a defragmented span.

This reducer can be used, for example, when the set of Keys for each Span being reduced is not expected to change, and therefore the keys from the first span encountered can be used without considering keys in subsequent spans.

type DefragmentingIter

type DefragmentingIter struct {
	// contains filtered or unexported fields
}

DefragmentingIter wraps a key span iterator, defragmenting physical fragmentation during iteration.

During flushes and compactions, keys applied over a span may be split at sstable boundaries. This fragmentation can produce internal key bounds that do not match any of the bounds ever supplied to a user operation. This physical fragmentation is necessary to avoid excessively wide sstables.

The defragmenting iterator undoes this physical fragmentation, joining spans with abutting bounds and equal state. The defragmenting iterator takes a DefragmentMethod to determine what is "equal state" for a span. The DefragmentMethod is a function type, allowing arbitrary comparisons between Span keys.

Seeking (SeekGE, SeekLT) poses an obstacle to defragmentation. A seek may land on a physical fragment in the middle of several fragments that must be defragmented. A seek first degfragments in the opposite direction of iteration to find the beginning of the defragmented span, and then defragments in the iteration direction, ensuring it's found a whole defragmented span.

func (*DefragmentingIter) Close

func (i *DefragmentingIter) Close() error

Close closes the underlying iterators.

func (*DefragmentingIter) Error

func (i *DefragmentingIter) Error() error

Error returns any accumulated error.

func (*DefragmentingIter) First

func (i *DefragmentingIter) First() *Span

First seeks the iterator to the first span and returns it.

func (*DefragmentingIter) Init

func (i *DefragmentingIter) Init(
	cmp base.Compare, iter FragmentIterator, equal DefragmentMethod, reducer DefragmentReducer,
)

Init initializes the defragmenting iter using the provided defragment method.

func (*DefragmentingIter) Last

func (i *DefragmentingIter) Last() *Span

Last seeks the iterator to the last span and returns it.

func (*DefragmentingIter) Next

func (i *DefragmentingIter) Next() *Span

Next advances to the next span and returns it.

func (*DefragmentingIter) Prev

func (i *DefragmentingIter) Prev() *Span

Prev steps back to the previous span and returns it.

func (*DefragmentingIter) SeekGE

func (i *DefragmentingIter) SeekGE(key []byte) *Span

SeekGE seeks the iterator to the first span with a start key greater than or equal to key and returns it.

func (*DefragmentingIter) SeekLT

func (i *DefragmentingIter) SeekLT(key []byte) *Span

SeekLT seeks the iterator to the last span with a start key less than key and returns it.

type FilterFunc

type FilterFunc func(in *Span, out *Span) (keep bool)

FilterFunc defines a transform from the input Span into the output Span. The function returns true if the Span should be returned by the iterator, and false if the Span should be skipped. The FilterFunc is permitted to mutate the output Span, for example, to elice certain keys, or update the Span's bounds if so desired. The output Span's Keys slice may be reused to reduce allocations.

type FragmentIterator

type FragmentIterator interface {
	// SeekGE moves the iterator to the first span whose start key is greater
	// than or equal to the given key.
	SeekGE(key []byte) *Span

	// SeekLT moves the iterator to the last span whose start key is less than
	// the given key.
	SeekLT(key []byte) *Span

	// First moves the iterator to the first span.
	First() *Span

	// Last moves the iterator to the last span.
	Last() *Span

	// Next moves the iterator to the next span.
	//
	// It is valid to call Next when the iterator is positioned before the first
	// key/value pair due to either a prior call to SeekLT or Prev which
	// returned an invalid span. It is not allowed to call Next when the
	// previous call to SeekGE, SeekPrefixGE or Next returned an invalid span.
	Next() *Span

	// Prev moves the iterator to the previous span.
	//
	// It is valid to call Prev when the iterator is positioned after the last
	// key/value pair due to either a prior call to SeekGE or Next which
	// returned an invalid span. It is not allowed to call Prev when the
	// previous call to SeekLT or Prev returned an invalid span.
	Prev() *Span

	// Error returns any accumulated error.
	Error() error

	// Close closes the iterator and returns any accumulated error. Exhausting
	// the iterator is not considered to be an error. It is valid to call Close
	// multiple times. Other methods should not be called after the iterator has
	// been closed.
	Close() error
}

FragmentIterator defines an iterator interface over spans. The spans surfaced by a FragmentIterator must be non-overlapping. This is achieved by fragmenting spans at overlap points (see Fragmenter).

A Span returned by a FragmentIterator is only valid until the next positioning method. Some implementations (eg, keyspan.Iter) may provide longer lifetimes but implementations need only guarantee stability until the next positioning method.

func Filter

func Filter(iter FragmentIterator, filter FilterFunc) FragmentIterator

Filter returns a new filteringIter that will filter the Spans from the provided child iterator using the provided FilterFunc.

func Truncate

func Truncate(
	cmp base.Compare, iter FragmentIterator, lower, upper []byte, start, end *base.InternalKey,
) FragmentIterator

Truncate creates a new iterator where every span in the supplied iterator is truncated to be contained within the range [lower, upper). If start and end are specified, filter out any spans that are completely outside those bounds.

type Fragmenter

type Fragmenter struct {
	Cmp    base.Compare
	Format base.FormatKey
	// Emit is called to emit a fragmented span and its keys. Every key defined
	// within the emitted Span applies to the entirety of the Span's key span.
	// Keys are ordered in decreasing order of their sequence numbers, and if
	// equal, decreasing order of key kind.
	Emit func(Span)
	// contains filtered or unexported fields
}

Fragmenter fragments a set of spans such that overlapping spans are split at their overlap points. The fragmented spans are output to the supplied Output function.

func (*Fragmenter) Add

func (f *Fragmenter) Add(s Span)

Add adds a span to the fragmenter. Spans may overlap and the fragmenter will internally split them. The spans must be presented in increasing start key order. That is, Add must be called with a series of spans like:

a---e
  c---g
  c-----i
         j---n
         j-l

We need to fragment the spans at overlap points. In the above example, we'd create:

a-c-e
  c-e-g
  c-e-g-i
         j-l-n
         j-l

The fragments need to be output sorted by start key, and for equal start keys, sorted by descending sequence number. This last part requires a mild bit of care as the fragments are not created in descending sequence number order.

Once a start key has been seen, we know that we'll never see a smaller start key and can thus flush all of the fragments that lie before that start key.

Walking through the example above, we start with:

a---e

Next we add [c,g) resulting in:

a-c-e
  c---g

The fragment [a,c) is flushed leaving the pending spans as:

c-e
c---g

The next span is [c,i):

c-e
c---g
c-----i

No fragments are flushed. The next span is [j,n):

c-e
c---g
c-----i
       j---n

The fragments [c,e), [c,g) and [c,i) are flushed. We sort these fragments by their end key, then split the fragments on the end keys:

c-e
c-e-g
c-e---i

The [c,e) fragments all get flushed leaving:

e-g
e---i

This process continues until there are no more fragments to flush.

WARNING: the slices backing Start, End, Keys, Key.Suffix and Key.Value are all retained after this method returns and should not be modified. This is safe for spans that are added from a memtable or batch. It is partially unsafe for a span read from an sstable. Specifically, the Keys slice of a Span returned during sstable iteration is only valid until the next iterator operation. The stability of the user keys depend on whether the block is prefix compressed, and in practice Pebble never prefix compresses range deletion and range key blocks, so these keys are stable. Because of this key stability, typically callers only need to perform a shallow clone of the Span before Add-ing it to the fragmenter.

func (*Fragmenter) Covers

func (f *Fragmenter) Covers(key base.InternalKey, snapshot uint64) bool

Covers returns true if the specified key is covered by one of the pending spans. The key must be consistent with the ordering of the spans. That is, it is invalid to specify a key here that is out of order with the span start keys passed to Add.

func (*Fragmenter) Empty

func (f *Fragmenter) Empty() bool

Empty returns true if all fragments added so far have finished flushing.

func (*Fragmenter) Finish

func (f *Fragmenter) Finish()

Finish flushes any remaining fragments to the output. It is an error to call this if any other spans will be added.

func (*Fragmenter) Start

func (f *Fragmenter) Start() []byte

Start returns the start key of the first span in the pending buffer, or nil if there are no pending spans. The start key of all pending spans is the same as that of the first one.

func (*Fragmenter) TruncateAndFlushTo

func (f *Fragmenter) TruncateAndFlushTo(key []byte)

TruncateAndFlushTo flushes all of the fragments with a start key <= key, truncating spans to the specified end key. Used during compaction to force emitting of spans which straddle an sstable boundary. Consider the scenario:

a---------k#10
     f#8
     f#7

/ Let's say the next user key after f is g. Calling TruncateAndFlushTo(g) will flush this span:

a-------g#10
     f#8
     f#7

And leave this one in f.pending:

g----k#10

WARNING: The fragmenter could hold on to the specified end key. Ensure it's a safe byte slice that could outlast the current sstable output, and one that will never be modified.

type Hooks

type Hooks interface {
	// SpanChanged is invoked by interleaving iterator whenever the Span
	// returned by InterleavaingIterator.Span changes. As the iterator passes
	// into or out of a Span, it invokes SpanChanged, passing the new Span.
	SpanChanged(*Span)
	// SkipPoint is invoked by the interleaving iterator whenever the iterator
	// encounters a point key covered by a Span. If SkipPoint returns true, the
	// interleaving iterator skips the point key without returning it. This is
	// used during range key iteration to skip over point keys 'masked' by range
	// keys.
	SkipPoint(userKey []byte) bool
}

A Hooks configures the interleaving iterator's behavior.

type InterleavingIter

type InterleavingIter struct {
	// contains filtered or unexported fields
}

InterleavingIter combines an iterator over point keys with an iterator over key spans.

Throughout Pebble, some keys apply at single discrete points within the user keyspace. Other keys apply over continuous spans of the user key space. Internally, iterators over point keys adhere to the base.InternalIterator interface, and iterators over spans adhere to the keyspan.FragmentIterator interface. The InterleavingIterator wraps a point iterator and span iterator, providing access to all the elements of both iterators.

The InterleavingIterator implements the point base.InternalIterator interface. After any of the iterator's methods return a key, a caller may call Span to retrieve the span covering the returned key, if any. A span is considered to 'cover' a returned key if the span's [start, end) bounds include the key's user key.

In addition to tracking the current covering span, InterleavingIter returns a special InternalKey at span start boundaries. Start boundaries are surfaced as a synthetic span marker: an InternalKey with the boundary as the user key, the infinite sequence number and a key kind selected from an arbitrary key the infinite sequence number and an arbitrary contained key's kind. Since which of the Span's key's kind is surfaced is undefined, the caller should not use the InternalKey's kind. The caller should only rely on the `Span` method for retrieving information about spanning keys. The interleaved synthetic keys have the infinite sequence number so that they're interleaved before any point keys with the same user key when iterating forward and after when iterating backward.

Interleaving the synthetic start key boundaries at the maximum sequence number provides an opportunity for the higher-level, public Iterator to observe the Span, even if no live points keys exist within the boudns of the Span.

When returning a synthetic marker key for a start boundary, InterleavingIter will truncate the span's start bound to the SeekGE or SeekPrefixGE search key. For example, a SeekGE("d") that finds a span [a, z) may return a synthetic span marker key `d#72057594037927935,21`.

If bounds have been applied to the iterator through SetBounds, InterleavingIter will truncate the bounds of spans returned through Span to the set bounds. The bounds returned through Span are not truncated by a SeekGE or SeekPrefixGE search key. Consider, for example SetBounds('c', 'e'), with an iterator containing the Span [a,z):

First()     = `c#72057594037927935,21`        Span() = [c,e)
SeekGE('d') = `d#72057594037927935,21`        Span() = [c,e)

InterleavedIter does not interleave synthetic markers for spans that do not contain any keys.

Hooks

InterelavingIter takes a Hooks parameter that may be used to configure the behavior of the iterator. See the documentation on the Hooks type.

All spans containing keys are exposed during iteration.

func (*InterleavingIter) Close

func (i *InterleavingIter) Close() error

Close implements (base.InternalIterator).Close.

func (*InterleavingIter) Error

func (i *InterleavingIter) Error() error

Error implements (base.InternalIterator).Error.

func (*InterleavingIter) First

func (i *InterleavingIter) First() (*base.InternalKey, []byte)

First implements (base.InternalIterator).First.

func (*InterleavingIter) Init

func (i *InterleavingIter) Init(
	cmp base.Compare,
	pointIter base.InternalIteratorWithStats,
	keyspanIter FragmentIterator,
	hooks Hooks,
	lowerBound, upperBound []byte,
)

Init initializes the InterleavingIter to interleave point keys from pointIter with key spans from keyspanIter.

The point iterator must already have the provided bounds. Init does not propagate the bounds down the iterator stack.

func (*InterleavingIter) Last

func (i *InterleavingIter) Last() (*base.InternalKey, []byte)

Last implements (base.InternalIterator).Last.

func (*InterleavingIter) Next

func (i *InterleavingIter) Next() (*base.InternalKey, []byte)

Next implements (base.InternalIterator).Next.

func (*InterleavingIter) Prev

func (i *InterleavingIter) Prev() (*base.InternalKey, []byte)

Prev implements (base.InternalIterator).Prev.

func (*InterleavingIter) ResetStats

func (i *InterleavingIter) ResetStats()

ResetStats implements InternalIteratorWithStats.

func (*InterleavingIter) SeekGE

func (i *InterleavingIter) SeekGE(key []byte, trySeekUsingNext bool) (*base.InternalKey, []byte)

SeekGE implements (base.InternalIterator).SeekGE.

If there exists a span with a start key ≤ the first matching point key, SeekGE will return a synthetic span marker key for the span. If this span's start key is less than key, the returned marker will be truncated to key. Note that this search-key truncation of the marker's key is not applied to the span returned by Span.

NB: In accordance with the base.InternalIterator contract:

i.lower ≤ key

func (*InterleavingIter) SeekLT

func (i *InterleavingIter) SeekLT(key []byte) (*base.InternalKey, []byte)

SeekLT implements (base.InternalIterator).SeekLT.

func (*InterleavingIter) SeekPrefixGE

func (i *InterleavingIter) SeekPrefixGE(
	prefix, key []byte, trySeekUsingNext bool,
) (*base.InternalKey, []byte)

SeekPrefixGE implements (base.InternalIterator).SeekPrefixGE.

If there exists a span with a start key ≤ the first matching point key, SeekPrefixGE will return a synthetic span marker key for the span. If this span's start key is less than key, the returned marker will be truncated to key. Note that this search-key truncation of the marker's key is not applied to the span returned by Span.

NB: In accordance with the base.InternalIterator contract:

i.lower ≤ key

func (*InterleavingIter) SetBounds

func (i *InterleavingIter) SetBounds(lower, upper []byte)

SetBounds implements (base.InternalIterator).SetBounds.

func (*InterleavingIter) Span

func (i *InterleavingIter) Span() *Span

Span returns the span covering the last key returned, if any. A span key is considered to 'cover' a key if the key falls within the span's user key bounds. The returned span is owned by the InterleavingIter. The caller is responsible for copying if stability is required.

Span will never return an invalid or empty span.

func (*InterleavingIter) Stats

Stats implements InternalIteratorWithStats.

func (*InterleavingIter) String

func (i *InterleavingIter) String() string

String implements (base.InternalIterator).String.

type InternalIteratorShim

type InternalIteratorShim struct {
	// contains filtered or unexported fields
}

InternalIteratorShim is a temporary iterator type used as a shim between keyspan.MergingIter and base.InternalIterator. It's used temporarily for range deletions during compactions, allowing range deletions to be interleaved by a compaction input iterator.

TODO(jackson): This type should be removed, and the usages converted to using an InterleavingIterator type that interleaves keyspan.Spans from a keyspan.FragmentIterator with point keys.

func (*InternalIteratorShim) Close

func (i *InternalIteratorShim) Close() error

Close implements (base.InternalIterator).Close.

func (*InternalIteratorShim) Error

func (i *InternalIteratorShim) Error() error

Error implements (base.InternalIterator).Error.

func (*InternalIteratorShim) First

func (i *InternalIteratorShim) First() (*base.InternalKey, []byte)

First implements (base.InternalIterator).First.

func (*InternalIteratorShim) Init

func (i *InternalIteratorShim) Init(cmp base.Compare, iters ...FragmentIterator)

Init initializes the internal iterator shim to merge the provided fragment iterators.

func (*InternalIteratorShim) Last

func (i *InternalIteratorShim) Last() (*base.InternalKey, []byte)

Last implements (base.InternalIterator).Last.

func (*InternalIteratorShim) Next

func (i *InternalIteratorShim) Next() (*base.InternalKey, []byte)

Next implements (base.InternalIterator).Next.

func (*InternalIteratorShim) Prev

func (i *InternalIteratorShim) Prev() (*base.InternalKey, []byte)

Prev implements (base.InternalIterator).Prev.

func (*InternalIteratorShim) SeekGE

func (i *InternalIteratorShim) SeekGE(
	key []byte, trySeekUsingNext bool,
) (*base.InternalKey, []byte)

SeekGE implements (base.InternalIterator).SeekGE.

func (*InternalIteratorShim) SeekLT

func (i *InternalIteratorShim) SeekLT(key []byte) (*base.InternalKey, []byte)

SeekLT implements (base.InternalIterator).SeekLT.

func (*InternalIteratorShim) SeekPrefixGE

func (i *InternalIteratorShim) SeekPrefixGE(
	prefix, key []byte, trySeekUsingNext bool,
) (*base.InternalKey, []byte)

SeekPrefixGE implements (base.InternalIterator).SeekPrefixGE.

func (*InternalIteratorShim) SetBounds

func (i *InternalIteratorShim) SetBounds(lower, upper []byte)

SetBounds implements (base.InternalIterator).SetBounds.

func (*InternalIteratorShim) Span

func (i *InternalIteratorShim) Span() *Span

Span returns the span containing the full set of keys over the key span at the current iterator position.

func (*InternalIteratorShim) String

func (i *InternalIteratorShim) String() string

String implements fmt.Stringer.

type Iter

type Iter struct {
	// contains filtered or unexported fields
}

Iter is an iterator over a set of fragmented spans.

func NewIter

func NewIter(cmp base.Compare, spans []Span) *Iter

NewIter returns a new iterator over a set of fragmented spans.

func (*Iter) Close

func (i *Iter) Close() error

Close implements FragmentIterator.Close.

func (*Iter) Count

func (i *Iter) Count() int

Count returns the number of spans contained by Iter.

func (*Iter) Error

func (i *Iter) Error() error

Error implements FragmentIterator.Error.

func (*Iter) First

func (i *Iter) First() *Span

First implements FragmentIterator.First.

func (*Iter) Init

func (i *Iter) Init(cmp base.Compare, spans []Span)

Init initializes an Iter with the provided spans.

func (*Iter) Last

func (i *Iter) Last() *Span

Last implements FragmentIterator.Last.

func (*Iter) Next

func (i *Iter) Next() *Span

Next implements FragmentIterator.Next.

func (*Iter) Prev

func (i *Iter) Prev() *Span

Prev implements FragmentIterator.Prev.

func (*Iter) SeekGE

func (i *Iter) SeekGE(key []byte) *Span

SeekGE implements FragmentIterator.SeekGE.

func (*Iter) SeekLT

func (i *Iter) SeekLT(key []byte) *Span

SeekLT implements FragmentIterator.SeekLT.

func (*Iter) String

func (i *Iter) String() string

type Key

type Key struct {
	// Trailer contains the key kind and sequence number.
	Trailer uint64
	// Suffix holds an optional suffix associated with the key. This is only
	// non-nil for RANGEKEYSET and RANGEKEYUNSET keys.
	Suffix []byte
	// Value holds a logical value associated with the Key. It is NOT the
	// internal value stored in a range key or range deletion block.  This is
	// only non-nil for RANGEKEYSET keys.
	Value []byte
}

Key represents a single key applied over a span of user keys. A Key is contained by a Span which specifies the span of user keys over which the Key is applied.

func (Key) Kind

func (k Key) Kind() base.InternalKeyKind

Kind returns the kind component of the key.

func (Key) SeqNum

func (k Key) SeqNum() uint64

SeqNum returns the sequence number component of the key.

type LevelIter

type LevelIter struct {
	// contains filtered or unexported fields
}

LevelIter provides a merged view of spans from sstables in a level. It takes advantage of level invariants to only have one sstable span block open at one time, opened using the newIter function passed in.

func (*LevelIter) Close

func (l *LevelIter) Close() error

Close implements keyspan.FragmentIterator.

func (*LevelIter) Error

func (l *LevelIter) Error() error

Error implements keyspan.FragmentIterator.

func (*LevelIter) First

func (l *LevelIter) First() *Span

First implements keyspan.FragmentIterator.

func (*LevelIter) Init

func (l *LevelIter) Init(
	opts SpanIterOptions,
	cmp base.Compare,
	newIter TableNewSpanIter,
	files manifest.LevelIterator,
	level manifest.Level,
	logger Logger,
	keyType manifest.KeyType,
)

Init initializes a LevelIter.

func (*LevelIter) Last

func (l *LevelIter) Last() *Span

Last implements keyspan.FragmentIterator.

func (*LevelIter) Next

func (l *LevelIter) Next() *Span

Next implements keyspan.FragmentIterator.

func (*LevelIter) Prev

func (l *LevelIter) Prev() *Span

Prev implements keyspan.FragmentIterator.

func (*LevelIter) SeekGE

func (l *LevelIter) SeekGE(key []byte) *Span

SeekGE implements keyspan.FragmentIterator.

func (*LevelIter) SeekLT

func (l *LevelIter) SeekLT(key []byte) *Span

SeekLT implements keyspan.FragmentIterator.

func (*LevelIter) String

func (l *LevelIter) String() string

String implements keyspan.FragmentIterator.

type Logger

type Logger interface {
	Infof(format string, args ...interface{})
	Fatalf(format string, args ...interface{})
}

Logger defines an interface for writing log messages.

type MergingIter

type MergingIter struct {
	// contains filtered or unexported fields
}

MergingIter merges spans across levels of the LSM, exposing an iterator over spans that yields sets of spans fragmented at unique user key boundaries.

A MergingIter is initialized with an arbitrary number of child iterators over fragmented spans. Each child iterator exposes fragmented key spans, such that overlapping keys are surfaced in a single Span. Key spans from one child iterator may overlap key spans from another child iterator arbitrarily.

Algorithm

The merging iterator wraps child iterators, merging and fragmenting spans across levels. The high-level algorithm is:

  1. Initialize the heap with bound keys from child iterators' spans.
  2. Find the next [or previous] two unique user keys' from bounds.
  3. Consider the span formed between the two unique user keys a candidate span.
  4. Determine if any of the child iterators' spans overlap the candidate span. 4a. If any of the child iterator's current bounds are end keys (during forward iteration) or start keys (during reverse iteration), then all the spans with that bound overlap the candidate span. 4b. Apply the configured transform, which may remove keys. 4b. If no spans overlap, forget the smallest (forward iteration) or largest (reverse iteration) unique user key and advance the iterators to the next unique user key. Start again from 3.

Detailed algorithm

Each level (i0, i1, ...) has a user-provided input FragmentIterator. The merging iterator steps through individual boundaries of the underlying spans separately. If the underlying FragmentIterator has fragments [a,b){#2,#1} [b,c){#1} the mergingIterLevel.{next,prev} step through:

(a, start), (b, end), (b, start), (c, end)

Note that (a, start) and (b, end) are observed ONCE each, despite two keys sharing those bounds. Also note that (b, end) and (b, start) are two distinct iterator positions of a mergingIterLevel.

The merging iterator maintains a heap (min during forward iteration, max during reverse iteration) containing the boundKeys. Each boundKey is a 3-tuple holding the bound user key, whether the bound is a start or end key and the set of keys from that level that have that bound. The heap orders based on the boundKey's user key only.

The merging iterator is responsible for merging spans across levels to determine which span is next, but it's also responsible for fragmenting overlapping spans. Consider the example:

       i0:     b---d e-----h
       i1:   a---c         h-----k
       i2:   a------------------------------p

fragments:   a-b-c-d-e-----h-----k----------p

None of the individual child iterators contain a span with the exact bounds [c,d), but the merging iterator must produce a span [c,d). To accomplish this, the merging iterator visits every span between unique boundary user keys. In the above example, this is:

[a,b), [b,c), [c,d), [d,e), [e, h), [h, k), [k, p)

The merging iterator first initializes the heap to prepare for iteration. The description below discusses the mechanics of forward iteration after a call to First, but the mechanics are similar for reverse iteration and other positioning methods.

During a call to First, the heap is initialized by seeking every mergingIterLevel to the first bound of the first fragment. In the above example, this seeks the child iterators to:

i0: (b, boundKindFragmentStart, [ [b,d) ])
i1: (a, boundKindFragmentStart, [ [a,c) ])
i2: (a, boundKindFragmentStart, [ [a,p) ])

After fixing up the heap, the root of the heap is a boundKey with the smallest user key ('a' in the example). Once the heap is setup for iteration in the appropriate direction and location, the merging iterator uses find{Next,Prev}FragmentSet to find the next/previous span bounds.

During forward iteration, the root of the heap's user key is the start key key of next merged span. findNextFragmentSet sets m.start to this user key. The heap may contain other boundKeys with the same user key if another level has a fragment starting or ending at the same key, so the findNextFragmentSet method pulls from the heap until it finds the first key greater than m.start. This key is used as the end key.

In the above example, this results in m.start = 'a', m.end = 'b' and child iterators in the following positions:

i0: (b, boundKindFragmentStart, [ [b,d) ])
i1: (c, boundKindFragmentEnd,   [ [a,c) ])
i2: (p, boundKindFragmentEnd,   [ [a,p) ])

With the user key bounds of the next merged span established, findNextFragmentSet must determine which, if any, fragments overlap the span. During forward iteration any child iterator that is now positioned at an end boundary has an overlapping span. (Justification: The child iterator's end boundary is ≥ m.end. The corresponding start boundary must be ≤ m.start since there were no other user keys between m.start and m.end. So the fragments associated with the iterator's current end boundary have start and end bounds such that start ≤ m.start < m.end ≤ end).

findNextFragmentSet iterates over the levels, collecting keys from any child iterators positioned at end boundaries. In the above example, i1 and i2 are positioned at end boundaries, so findNextFragmentSet collects the keys of [a,c) and [a,p). These spans contain the merging iterator's [m.start, m.end) span, but they may also extend beyond the m.start and m.end. The merging iterator returns the keys with the merging iter's m.start and m.end bounds, preserving the underlying keys' sequence numbers, key kinds and values.

A MergingIter is configured with a Transform that's applied to the span before surfacing it to the iterator user. A Transform may remove keys arbitrarily, but it may not modify the values themselves.

It may be the case that findNextFragmentSet finds no levels positioned at end boundaries, or that there are no spans remaining after applying a transform, in which case the span [m.start, m.end) overlaps with nothing. In this case findNextFragmentSet loops, repeating the above process again until it finds a span that does contain keys.

Memory safety

The FragmentIterator interface only guarantees stability of a Span and its associated slices until the next positioning method is called. Adjacent Spans may be contained in different sstables, requring the FragmentIterator implementation to close one sstable, releasing its memory, before opening the next. Most of the state used by the MergingIter is derived from spans at current child iterator positions only, ensuring state is stable. The one exception is the start bound during forward iteration and the end bound during reverse iteration.

If the heap root originates from an end boundary when findNextFragmentSet begins, a Next on the heap root level may invalidate the end boundary. To accommodate this, find{Next,Prev}FragmentSet copy the initial boundary if the subsequent Next/Prev would move to the next span.

func (*MergingIter) AddLevel

func (m *MergingIter) AddLevel(iter FragmentIterator)

AddLevel adds a new level to the bottom of the merging iterator. AddLevel must be called after Init and before any other method.

func (*MergingIter) Close

func (m *MergingIter) Close() error

Close closes the iterator, releasing all acquired resources.

func (*MergingIter) DebugString

func (m *MergingIter) DebugString() string

DebugString returns a string representing the current internal state of the merging iterator and its heap for debugging purposes.

func (*MergingIter) Error

func (m *MergingIter) Error() error

Error returns any accumulated error.

func (*MergingIter) First

func (m *MergingIter) First() *Span

First seeks the iterator to the first span.

func (*MergingIter) Init

func (m *MergingIter) Init(cmp base.Compare, transformer Transformer, iters ...FragmentIterator)

Init initializes the merging iterator with the provided fragment iterators.

func (*MergingIter) Last

func (m *MergingIter) Last() *Span

Last seeks the iterator to the last span.

func (*MergingIter) Next

func (m *MergingIter) Next() *Span

Next advances the iterator to the next span.

func (*MergingIter) Prev

func (m *MergingIter) Prev() *Span

Prev advances the iterator to the previous span.

func (*MergingIter) SeekGE

func (m *MergingIter) SeekGE(key []byte) *Span

SeekGE moves the iterator to the first span with a start key greater than or equal to key.

func (*MergingIter) SeekLT

func (m *MergingIter) SeekLT(key []byte) *Span

SeekLT moves the iterator to the last span with a start key less than key.

func (*MergingIter) String

func (m *MergingIter) String() string

String implements fmt.Stringer.

type Span

type Span struct {
	// Start and End encode the user key range of all the contained items, with
	// an inclusive start key and exclusive end key. Both Start and End must be
	// non-nil, or both nil if representing an invalid Span.
	Start, End []byte
	// Keys holds the set of keys applied over the [Start, End) user key range.
	// Keys is sorted by (SeqNum, Kind) descending. If SeqNum and Kind are
	// equal, the order of Keys is undefined. Keys may be empty, even if Start
	// and End are non-nil.
	//
	// Keys are a decoded representation of the internal keys stored in batches
	// or sstable blocks. A single internal key in a range key block may produce
	// several decoded Keys.
	Keys []Key
}

Span represents a set of keys over a span of user key space. All of the keys within a Span are applied across the span's key span indicated by Start and End. Each internal key applied over the user key span appears as a separate Key, with its own kind and sequence number. Optionally, each Key may also have a Suffix and/or Value.

Note that the start user key is inclusive and the end user key is exclusive.

Currently the only supported key kinds are:

RANGEDEL, RANGEKEYSET, RANGEKEYUNSET, RANGEKEYDEL.

func Get

func Get(cmp base.Compare, iter FragmentIterator, key []byte) *Span

Get returns the newest span that contains the target key. If no span contains the target key, an empty span is returned. The snapshot parameter controls the visibility of spans (only spans older than the snapshot sequence number are visible). The iterator must contain fragmented spans: no span may overlap another.

func ParseSpan

func ParseSpan(input string) Span

ParseSpan parses the string representation of a Span. It's intended for tests. ParseSpan panics if passed a malformed span representation.

func SeekGE

func SeekGE(cmp base.Compare, iter FragmentIterator, key []byte) *Span

SeekGE seeks to the span that contains the target key or the first span past the target key.

func SeekLE

func SeekLE(cmp base.Compare, iter FragmentIterator, key []byte) *Span

SeekLE seeks to the span that contains or is before the target key.

func (*Span) Contains

func (s *Span) Contains(cmp base.Compare, key []byte) bool

Contains returns true if the specified key resides within the span's bounds.

func (Span) Covers

func (s Span) Covers(seqNum uint64) bool

Covers returns true if the span covers keys at seqNum.

func (*Span) CoversAt

func (s *Span) CoversAt(snapshot, seqNum uint64) bool

CoversAt returns true if the span contains a key that is visible at the provided snapshot sequence number, and that key's sequence number is higher than seqNum.

Keys with sequence numbers with the batch bit set are treated as always visible.

func (*Span) DeepClone

func (s *Span) DeepClone() Span

DeepClone clones the span, creating copies of all contained slices. DeepClone is intended for non-production code paths like tests, the level checker, etc because it is allocation heavy.

func (*Span) Empty

func (s *Span) Empty() bool

Empty returns true if the span does not contain any keys. An empty span may still be Valid. A non-empty span must be Valid.

An Empty span may be produced by Visible, or be produced by iterators in order to surface the gaps between keys.

func (*Span) LargestKey

func (s *Span) LargestKey() base.InternalKey

LargestKey returns the largest internal key defined by the span's keys. The returned key will always be a "sentinel key" at the end boundary. The "sentinel key" models the exclusive end boundary by returning an InternalKey with the maximal sequence number, ensuring all InternalKeys with the same user key sort after the sentinel key.

It panics if the span contains no keys.

func (*Span) LargestSeqNum

func (s *Span) LargestSeqNum() uint64

LargestSeqNum returns the largest sequence number of a key contained within the span. It panics if the span contains no keys.

func (Span) Pretty

func (s Span) Pretty(f base.FormatKey) fmt.Formatter

Pretty returns a formatter for the span.

func (*Span) ShallowClone

func (s *Span) ShallowClone() Span

ShallowClone returns the span with a Keys slice owned by the span itself. None of the key byte slices are cloned (see Span.DeepClone).

func (*Span) SmallestKey

func (s *Span) SmallestKey() base.InternalKey

SmallestKey returns the smallest internal key defined by the span's keys. It panics if the span contains no keys.

func (*Span) SmallestSeqNum

func (s *Span) SmallestSeqNum() uint64

SmallestSeqNum returns the smallest sequence number of a key contained within the span. It panics if the span contains no keys.

func (Span) String

func (s Span) String() string

String returns a string representation of the span.

func (*Span) Valid

func (s *Span) Valid() bool

Valid returns true if the span is defined.

func (Span) Visible

func (s Span) Visible(snapshot uint64) Span

Visible returns a span with the subset of keys visible at the provided sequence number.

Visible may incur an allocation, so callers should prefer targeted, non-allocating methods when possible.

func (*Span) VisibleAt

func (s *Span) VisibleAt(snapshot uint64) bool

VisibleAt returns true if the span contains a key visible at the provided snapshot. Keys with sequence numbers with the batch bit set are treated as always visible.

type SpanIterOptions

type SpanIterOptions struct {
	// RangeKeyFilters can be used to avoid scanning tables and blocks in tables
	// when iterating over range keys.
	RangeKeyFilters []base.BlockPropertyFilter
}

SpanIterOptions is a subset of IterOptions that are necessary to instantiate per-sstable span iterators.

type TableNewSpanIter

type TableNewSpanIter func(file *manifest.FileMetadata, iterOptions *SpanIterOptions) (FragmentIterator, error)

TableNewSpanIter creates a new iterator for spans for the given file.

type Transformer

type Transformer interface {
	// Transform takes a Span as input and writes the transformed Span to the
	// provided output *Span pointer. The output Span's Keys slice may be reused
	// by Transform to reduce allocations.
	Transform(cmp base.Compare, in Span, out *Span) error
}

Transformer defines a transformation to be applied to a Span.

type TransformerFunc

type TransformerFunc func(base.Compare, Span, *Span) error

The TransformerFunc type is an adapter to allow the use of ordinary functions as Transformers. If f is a function with the appropriate signature, TransformerFunc(f) is a Transformer that calls f.

func (TransformerFunc) Transform

func (tf TransformerFunc) Transform(cmp base.Compare, in Span, out *Span) error

Transform calls f(cmp, in, out).

Jump to

Keyboard shortcuts

? : This menu
/ : Search site
f or F : Jump to
y or Y : Canonical URL