tsi1

package
v1.9.5 Latest Latest
Warning

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

Go to latest
Published: Sep 30, 2021 License: MIT Imports: 34 Imported by: 29

Documentation

Overview

Package tsi1 provides a memory-mapped index implementation that supports high cardinality series.

Overview

The top-level object in tsi1 is the Index. It is the primary access point from the rest of the system. The Index is composed of LogFile and IndexFile objects.

Log files are small write-ahead log files that record new series immediately in the order that they are received. The data within the file is indexed in-memory so it can be quickly accessed. When the system is restarted, this log file is replayed and the in-memory representation is rebuilt.

Index files also contain series information, however, they are highly indexed so that reads can be performed quickly. Index files are built through a process called compaction where a log file or multiple index files are merged together.

Operations

The index can perform many tasks related to series, measurement, & tag data. All data is inserted by adding a series to the index. When adding a series, the measurement, tag keys, and tag values are all extracted and indexed separately.

Once a series has been added, it can be removed in several ways. First, the individual series can be removed. Second, it can be removed as part of a bulk operation by deleting the entire measurement.

The query engine needs to be able to look up series in a variety of ways such as by measurement name, by tag value, or by using regular expressions. The index provides an API to iterate over subsets of series and perform set operations such as unions and intersections.

Log File Layout

The write-ahead file that series initially are inserted into simply appends all new operations sequentially. It is simply composed of a series of log entries. An entry contains a flag to specify the operation type, the measurement name, the tag set, and a checksum.

┏━━━━━━━━━LogEntry━━━━━━━━━┓
┃ ┌──────────────────────┐ ┃
┃ │         Flag         │ ┃
┃ ├──────────────────────┤ ┃
┃ │     Measurement      │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Key/Value       │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Key/Value       │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Key/Value       │ ┃
┃ ├──────────────────────┤ ┃
┃ │       Checksum       │ ┃
┃ └──────────────────────┘ ┃
┗━━━━━━━━━━━━━━━━━━━━━━━━━━┛

When the log file is replayed, if the checksum is incorrect or the entry is incomplete (because of a partially failed write) then the log is truncated.

Index File Layout

The index file is composed of 3 main block types: one series block, one or more tag blocks, and one measurement block. At the end of the index file is a trailer that records metadata such as the offsets to these blocks.

Series Block Layout

The series block stores raw series keys in sorted order. It also provides hash indexes so that series can be looked up quickly. Hash indexes are inserted periodically so that memory size is limited at write time. Once all the series and hash indexes have been written then a list of index entries are written so that hash indexes can be looked up via binary search.

The end of the block contains two HyperLogLog++ sketches which track the estimated number of created series and deleted series. After the sketches is a trailer which contains metadata about the block.

┏━━━━━━━SeriesBlock━━━━━━━━┓
┃ ┌──────────────────────┐ ┃
┃ │      Series Key      │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Series Key      │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Series Key      │ ┃
┃ ├──────────────────────┤ ┃
┃ │                      │ ┃
┃ │      Hash Index      │ ┃
┃ │                      │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Series Key      │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Series Key      │ ┃
┃ ├──────────────────────┤ ┃
┃ │      Series Key      │ ┃
┃ ├──────────────────────┤ ┃
┃ │                      │ ┃
┃ │      Hash Index      │ ┃
┃ │                      │ ┃
┃ ├──────────────────────┤ ┃
┃ │    Index Entries     │ ┃
┃ ├──────────────────────┤ ┃
┃ │     HLL Sketches     │ ┃
┃ ├──────────────────────┤ ┃
┃ │       Trailer        │ ┃
┃ └──────────────────────┘ ┃
┗━━━━━━━━━━━━━━━━━━━━━━━━━━┛

Tag Block Layout

After the series block is one or more tag blocks. One of these blocks exists for every measurement in the index file. The block is structured as a sorted list of values for each key and then a sorted list of keys. Each of these lists has their own hash index for fast direct lookups.

┏━━━━━━━━Tag Block━━━━━━━━━┓
┃ ┌──────────────────────┐ ┃
┃ │        Value         │ ┃
┃ ├──────────────────────┤ ┃
┃ │        Value         │ ┃
┃ ├──────────────────────┤ ┃
┃ │        Value         │ ┃
┃ ├──────────────────────┤ ┃
┃ │                      │ ┃
┃ │      Hash Index      │ ┃
┃ │                      │ ┃
┃ └──────────────────────┘ ┃
┃ ┌──────────────────────┐ ┃
┃ │        Value         │ ┃
┃ ├──────────────────────┤ ┃
┃ │        Value         │ ┃
┃ ├──────────────────────┤ ┃
┃ │                      │ ┃
┃ │      Hash Index      │ ┃
┃ │                      │ ┃
┃ └──────────────────────┘ ┃
┃ ┌──────────────────────┐ ┃
┃ │         Key          │ ┃
┃ ├──────────────────────┤ ┃
┃ │         Key          │ ┃
┃ ├──────────────────────┤ ┃
┃ │                      │ ┃
┃ │      Hash Index      │ ┃
┃ │                      │ ┃
┃ └──────────────────────┘ ┃
┃ ┌──────────────────────┐ ┃
┃ │       Trailer        │ ┃
┃ └──────────────────────┘ ┃
┗━━━━━━━━━━━━━━━━━━━━━━━━━━┛

Each entry for values contains a sorted list of offsets for series keys that use that value. Series iterators can be built around a single tag key value or multiple iterators can be merged with set operators such as union or intersection.

Measurement block

The measurement block stores a sorted list of measurements, their associated series offsets, and the offset to their tag block. This allows all series for a measurement to be traversed quickly and it allows fast direct lookups of measurements and their tags.

This block also contains HyperLogLog++ sketches for new and deleted measurements.

┏━━━━Measurement Block━━━━━┓
┃ ┌──────────────────────┐ ┃
┃ │     Measurement      │ ┃
┃ ├──────────────────────┤ ┃
┃ │     Measurement      │ ┃
┃ ├──────────────────────┤ ┃
┃ │     Measurement      │ ┃
┃ ├──────────────────────┤ ┃
┃ │                      │ ┃
┃ │      Hash Index      │ ┃
┃ │                      │ ┃
┃ ├──────────────────────┤ ┃
┃ │     HLL Sketches     │ ┃
┃ ├──────────────────────┤ ┃
┃ │       Trailer        │ ┃
┃ └──────────────────────┘ ┃
┗━━━━━━━━━━━━━━━━━━━━━━━━━━┛

Manifest file

The index is simply an ordered set of log and index files. These files can be merged together or rewritten but their order must always be the same. This is because series, measurements, & tags can be marked as deleted (aka tombstoned) and this action needs to be tracked in time order.

Whenever the set of active files is changed, a manifest file is written to track the set. The manifest specifies the ordering of files and, on startup, all files not in the manifest are removed from the index directory.

Compacting index files

Compaction is the process of taking files and merging them together into a single file. There are two stages of compaction within TSI.

First, once log files exceed a size threshold then they are compacted into an index file. This threshold is relatively small because log files must maintain their index in the heap which TSI tries to avoid. Small log files are also very quick to convert into an index file so this is done aggressively.

Second, once a contiguous set of index files exceed a factor (e.g. 10x) then they are all merged together into a single index file and the old files are discarded. Because all blocks are written in sorted order, the new index file can be streamed and minimize memory use.

Concurrency

Index files are immutable so they do not require fine grained locks, however, compactions require that we track which files are in use so they are not discarded too soon. This is done by using reference counting with file sets.

A file set is simply an ordered list of index files. When the current file set is obtained from the index, a counter is incremented to track its usage. Once the user is done with the file set, it is released and the counter is decremented. A file cannot be removed from the file system until this counter returns to zero.

Besides the reference counting, there are no other locking mechanisms when reading or writing index files. Log files, however, do require a lock whenever they are accessed. This is another reason to minimize log file size.

Index

Constants

View Source
const (
	// IndexFile trailer fields
	IndexFileVersionSize = 2

	// IndexFileTrailerSize is the size of the trailer. Currently 82 bytes.
	IndexFileTrailerSize = IndexFileVersionSize +
		8 + 8 +
		8 + 8 +
		8 + 8 +
		8 + 8 +
		8 + 8 +
		0
)

IndexFile field size constants.

View Source
const (
	LogEntrySeriesTombstoneFlag      = 0x01
	LogEntryMeasurementTombstoneFlag = 0x02
	LogEntryTagKeyTombstoneFlag      = 0x04
	LogEntryTagValueTombstoneFlag    = 0x08
)

Log entry flag constants.

View Source
const (
	MeasurementTombstoneFlag   = 0x01
	MeasurementSeriesIDSetFlag = 0x02
)

Measurement flag constants.

View Source
const (
	// 1 byte offset for the block to ensure non-zero offsets.
	MeasurementFillSize = 1

	// Measurement trailer fields
	MeasurementTrailerSize = 0 +
		2 +
		8 + 8 +
		8 + 8 +
		8 + 8 +
		8 + 8 // tombstone measurement sketch offset/size

	// Measurement key block fields.
	MeasurementNSize      = 8
	MeasurementOffsetSize = 8

	SeriesIDSize = 8
)

Measurement field size constants.

View Source
const (
	LogFileExt   = ".tsl"
	IndexFileExt = ".tsi"

	CompactingExt = ".compacting"
)

File extensions.

View Source
const (
	TagValueTombstoneFlag   = 0x01
	TagValueSeriesIDSetFlag = 0x02
)

Tag value flag constants.

View Source
const (
	// TagBlock key block fields.
	TagKeyNSize      = 8
	TagKeyOffsetSize = 8

	// TagBlock value block fields.
	TagValueNSize      = 8
	TagValueOffsetSize = 8
)

TagBlock variable size constants.

View Source
const FileSignature = "TSI1"

FileSignature represents a magic number at the header of the index file.

View Source
const IndexFileVersion = 1

IndexFileVersion is the current TSI1 index file version.

View Source
const IndexName = tsdb.TSI1IndexName

IndexName is the name of the index.

View Source
const LoadFactor = 80

LoadFactor is the fill percent for RHH indexes.

View Source
const ManifestFileName = "MANIFEST"

ManifestFileName is the name of the index manifest file.

View Source
const MaxIndexFileSize = 4 * (1 << 30)

MaxIndexFileSize is the maximum expected size of an index file.

View Source
const MaxIndexMergeCount = 2

MaxIndexMergeCount is the maximum number of files that can be merged together at once.

View Source
const MeasurementBlockVersion = 1

MeasurementBlockVersion is the version of the measurement block.

View Source
const TagBlockTrailerSize = 0 +
	8 + 8 +
	8 + 8 +
	8 + 8 +
	8 +

	2 // version

TagBlockTrailerSize is the total size of the on-disk trailer.

View Source
const TagBlockVersion = 1

TagBlockVersion is the version of the tag block.

View Source
const (
	TagKeyTombstoneFlag = 0x01
)

Tag key flag constants.

View Source
const Version = 1

Version is the current version of the TSI index.

Variables

View Source
var (
	ErrInvalidIndexFile            = errors.New("invalid index file")
	ErrUnsupportedIndexFileVersion = errors.New("unsupported index file version")
)

IndexFile errors.

View Source
var (
	ErrUnsupportedMeasurementBlockVersion = errors.New("unsupported measurement block version")
	ErrMeasurementBlockSizeMismatch       = errors.New("measurement block size mismatch")
)

Measurement errors.

View Source
var (
	ErrUnsupportedTagBlockVersion = errors.New("unsupported tag block version")
	ErrTagBlockSizeMismatch       = errors.New("tag block size mismatch")
)

TagBlock errors.

View Source
var DefaultCompactionLevels = []CompactionLevel{
	{M: 0, K: 0},
	{M: 1 << 25, K: 6},
	{M: 1 << 25, K: 6},
	{M: 1 << 26, K: 6},
	{M: 1 << 27, K: 6},
	{M: 1 << 28, K: 6},
	{M: 1 << 29, K: 6},
	{M: 1 << 30, K: 6},
}

DefaultCompactionLevels is the default settings used by the index.

View Source
var DefaultPartitionN uint64 = 8

DefaultPartitionN determines how many shards the index will be partitioned into.

NOTE: Currently, this must not be change once a database is created. Further, it must also be a power of 2.

View Source
var DisableCompactions = func() IndexOption {
	return func(i *Index) {
		i.disableCompactions = true
	}
}

DisableCompactions disables compactions on the Index.

View Source
var DisableFsync = func() IndexOption {
	return func(i *Index) {
		i.disableFsync = true
	}
}

DisableFsync disables flushing and syncing of underlying files. Primarily this impacts the LogFiles. This option can be set when working with the index in an offline manner, for cases where a hard failure can be overcome by re-running the tooling.

View Source
var ErrCompactionInterrupted = errors.New("tsi1: compaction interrupted")

ErrCompactionInterrupted is returned if compactions are disabled or an index is closed while a compaction is occurring.

View Source
var ErrIncompatibleVersion = errors.New("incompatible tsi1 index MANIFEST")

ErrIncompatibleVersion is returned when attempting to read from an incompatible tsi1 manifest file.

View Source
var (
	ErrLogEntryChecksumMismatch = errors.New("log entry checksum mismatch")
)

Log errors.

View Source
var WithLogFileBufferSize = func(sz int) IndexOption {
	return func(i *Index) {
		if sz > 1<<17 {
			sz = 1 << 17
		} else if sz < 1<<12 {
			sz = 1 << 12
		}
		i.logfileBufferSize = sz
	}
}

WithLogFileBufferSize sets the size of the buffer used within LogFiles. Typically appending an entry to a LogFile involves writing 11 or 12 bytes, so depending on how many new series are being created within a batch, it may be appropriate to set this.

View Source
var WithLogger = func(l zap.Logger) IndexOption {
	return func(i *Index) {
		i.logger = l.With(zap.String("index", "tsi"))
	}
}

WithLogger sets the logger for the Index.

View Source
var WithMaximumLogFileAge = func(dur time.Duration) IndexOption {
	return func(i *Index) {
		i.maxLogFileAge = dur
	}
}
View Source
var WithMaximumLogFileSize = func(size int64) IndexOption {
	return func(i *Index) {
		i.maxLogFileSize = size
	}
}

WithMaximumLogFileSize sets the maximum size of LogFiles before they're compacted into IndexFiles.

View Source
var WithPath = func(path string) IndexOption {
	return func(i *Index) {
		i.path = path
	}
}

WithPath sets the root path of the Index

View Source
var WithSeriesIDCacheSize = func(sz int) IndexOption {
	return func(i *Index) {
		i.tagValueCacheSize = sz
	}
}

WithSeriesIDCacheSize sets the size of the series id set cache. If set to 0, then the cache is disabled.

Functions

func FormatIndexFileName

func FormatIndexFileName(id, level int) string

FormatIndexFileName generates an index filename for the given index.

func FormatLogFileName

func FormatLogFileName(id int) string

FormatLogFileName generates a log filename for the given index.

func IsIndexDir added in v1.5.0

func IsIndexDir(path string) (bool, error)

IsIndexDir returns true if directory contains at least one partition directory.

func IsPartitionDir added in v1.5.0

func IsPartitionDir(path string) (bool, error)

IsPartitionDir returns true if directory contains a MANIFEST file.

func NewTSDBMeasurementIteratorAdapter added in v1.5.0

func NewTSDBMeasurementIteratorAdapter(itr MeasurementIterator) tsdb.MeasurementIterator

NewTSDBMeasurementIteratorAdapter return an iterator which implements tsdb.MeasurementIterator.

func NewTSDBTagKeyIteratorAdapter added in v1.5.0

func NewTSDBTagKeyIteratorAdapter(itr TagKeyIterator) tsdb.TagKeyIterator

NewTSDBTagKeyIteratorAdapter return an iterator which implements tsdb.TagKeyIterator.

func NewTSDBTagValueIteratorAdapter added in v1.5.0

func NewTSDBTagValueIteratorAdapter(itr TagValueIterator) tsdb.TagValueIterator

NewTSDBTagValueIteratorAdapter return an iterator which implements tsdb.TagValueIterator.

func ParseFilename

func ParseFilename(name string) (level, id int)

ParseFilename extracts the numeric id from a log or index file path. Returns 0 if it cannot be parsed.

Types

type CompactionLevel

type CompactionLevel struct {
	// Bloom filter bit size & hash count
	M uint64 `json:"m,omitempty"`
	K uint64 `json:"k,omitempty"`
}

CompactionLevel represents a grouping of index files based on bloom filter settings. By having the same bloom filter settings, the filters can be merged and evaluated at a higher level.

type File

type File interface {
	Close() error
	Path() string

	ID() int
	Level() int

	Measurement(name []byte) MeasurementElem
	MeasurementIterator() MeasurementIterator
	MeasurementHasSeries(ss *tsdb.SeriesIDSet, name []byte) bool

	TagKey(name, key []byte) TagKeyElem
	TagKeyIterator(name []byte) TagKeyIterator

	TagValue(name, key, value []byte) TagValueElem
	TagValueIterator(name, key []byte) TagValueIterator

	// Series iteration.
	MeasurementSeriesIDIterator(name []byte) tsdb.SeriesIDIterator
	TagKeySeriesIDIterator(name, key []byte) (tsdb.SeriesIDIterator, error)
	TagValueSeriesIDSet(name, key, value []byte) (*tsdb.SeriesIDSet, error)

	// Sketches for cardinality estimation
	MeasurementsSketches() (s, t estimator.Sketch, err error)
	SeriesSketches() (s, t estimator.Sketch, err error)

	// Bitmap series existance.
	SeriesIDSet() (*tsdb.SeriesIDSet, error)
	TombstoneSeriesIDSet() (*tsdb.SeriesIDSet, error)

	// Reference counting.
	Retain()
	Release()

	// Size of file on disk
	Size() int64
	// contains filtered or unexported methods
}

File represents a log or index file.

type FileSet

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

FileSet represents a collection of files.

func NewFileSet

func NewFileSet(files []File) *FileSet

NewFileSet returns a new instance of FileSet.

func (FileSet) Close

func (fs FileSet) Close() error

Close closes all the files in the file set.

func (*FileSet) Files

func (fs *FileSet) Files() []File

Files returns all files in the set.

func (*FileSet) HasTagKey

func (fs *FileSet) HasTagKey(name, key []byte) bool

HasTagKey returns true if the tag key exists.

func (*FileSet) HasTagValue

func (fs *FileSet) HasTagValue(name, key, value []byte) bool

HasTagValue returns true if the tag value exists.

func (*FileSet) LastContiguousIndexFilesByLevel

func (fs *FileSet) LastContiguousIndexFilesByLevel(level int) []*IndexFile

LastContiguousIndexFilesByLevel returns the last contiguous files by level. These can be used by the compaction scheduler.

func (*FileSet) MaxID

func (fs *FileSet) MaxID() int

MaxID returns the highest file identifier.

func (*FileSet) Measurement

func (fs *FileSet) Measurement(name []byte) MeasurementElem

Measurement returns a measurement by name.

func (*FileSet) MeasurementIterator

func (fs *FileSet) MeasurementIterator() MeasurementIterator

MeasurementIterator returns an iterator over all measurements in the index.

func (*FileSet) MeasurementSeriesIDIterator added in v1.5.0

func (fs *FileSet) MeasurementSeriesIDIterator(name []byte) tsdb.SeriesIDIterator

MeasurementSeriesIDIterator returns a series iterator for a measurement.

func (*FileSet) MeasurementTagKeysByExpr

func (fs *FileSet) MeasurementTagKeysByExpr(name []byte, expr influxql.Expr) (map[string]struct{}, error)

MeasurementTagKeysByExpr extracts the tag keys wanted by the expression.

func (*FileSet) MeasurementsSketches

func (fs *FileSet) MeasurementsSketches() (estimator.Sketch, estimator.Sketch, error)

MeasurementsSketches returns the merged measurement sketches for the FileSet.

func (*FileSet) MustReplace

func (fs *FileSet) MustReplace(oldFiles []File, newFile File) *FileSet

MustReplace swaps a list of files for a single file and returns a new file set. The caller should always guarantee that the files exist and are contiguous.

func (*FileSet) PrependLogFile added in v1.4.0

func (fs *FileSet) PrependLogFile(f *LogFile) *FileSet

PrependLogFile returns a new file set with f added at the beginning. Filters do not need to be rebuilt because log files have no bloom filter.

func (*FileSet) Release

func (fs *FileSet) Release()

Release removes a reference count from all files.

func (*FileSet) Retain

func (fs *FileSet) Retain()

Retain adds a reference count to all files.

func (*FileSet) SeriesSketches

func (fs *FileSet) SeriesSketches() (estimator.Sketch, estimator.Sketch, error)

SeriesSketches returns the merged measurement sketches for the FileSet.

func (*FileSet) Size added in v1.5.0

func (fs *FileSet) Size() int64

Size returns the on-disk size of the FileSet.

func (*FileSet) TagKeyIterator

func (fs *FileSet) TagKeyIterator(name []byte) TagKeyIterator

TagKeyIterator returns an iterator over all tag keys for a measurement.

func (*FileSet) TagKeySeriesIDIterator added in v1.5.0

func (fs *FileSet) TagKeySeriesIDIterator(name, key []byte) (tsdb.SeriesIDIterator, error)

TagKeySeriesIDIterator returns a series iterator for all values across a single key.

func (*FileSet) TagValueIterator

func (fs *FileSet) TagValueIterator(name, key []byte) TagValueIterator

TagValueIterator returns a value iterator for a tag key.

func (*FileSet) TagValueSeriesIDIterator added in v1.5.0

func (fs *FileSet) TagValueSeriesIDIterator(name, key, value []byte) (tsdb.SeriesIDIterator, error)

TagValueSeriesIDIterator returns a series iterator for a single tag value.

type Files

type Files []File

func (Files) IDs

func (a Files) IDs() []int

type Index

type Index struct {

	// Number of partitions used by the index.
	PartitionN uint64
	// contains filtered or unexported fields
}

Index represents a collection of layered index files and WAL.

func NewIndex

func NewIndex(sfile *tsdb.SeriesFile, database string, options ...IndexOption) *Index

NewIndex returns a new instance of Index.

func (*Index) Bytes added in v1.6.0

func (i *Index) Bytes() int

Bytes estimates the memory footprint of this Index, in bytes.

func (*Index) Close

func (i *Index) Close() error

Close closes the index.

func (*Index) Compact

func (i *Index) Compact()

Compact requests a compaction of partitions.

func (*Index) CreateSeriesIfNotExists

func (i *Index) CreateSeriesIfNotExists(key, name []byte, tags models.Tags, tracker tsdb.StatsTracker) error

CreateSeriesIfNotExists creates a series if it doesn't exist or is deleted.

func (*Index) CreateSeriesListIfNotExists

func (i *Index) CreateSeriesListIfNotExists(keys, names [][]byte, tagsSlice []models.Tags, tracker tsdb.StatsTracker) error

CreateSeriesListIfNotExists creates a list of series if they doesn't exist in bulk.

func (*Index) Database

func (i *Index) Database() string

Database returns the name of the database the index was initialized with.

func (*Index) DisableCompactions added in v1.5.0

func (i *Index) DisableCompactions()

func (*Index) DiskSizeBytes added in v1.5.0

func (i *Index) DiskSizeBytes() int64

DiskSizeBytes returns the size of the index on disk.

func (*Index) DropMeasurement

func (i *Index) DropMeasurement(name []byte) error

DropMeasurement deletes a measurement from the index. It returns the first error encountered, if any.

func (*Index) DropMeasurementIfSeriesNotExist added in v1.5.0

func (i *Index) DropMeasurementIfSeriesNotExist(name []byte) (bool, error)

DropMeasurementIfSeriesNotExist drops a measurement only if there are no more series for the measurment.

func (*Index) DropSeries

func (i *Index) DropSeries(seriesID uint64, key []byte, cascade bool) error

DropSeries drops the provided series from the index. If cascade is true and this is the last series to the measurement, the measurment will also be dropped.

func (*Index) DropSeriesGlobal added in v1.5.5

func (i *Index) DropSeriesGlobal(key []byte) error

DropSeriesGlobal is a no-op on the tsi1 index.

func (*Index) DropSeriesList added in v1.8.1

func (i *Index) DropSeriesList(seriesIDs []uint64, keys [][]byte, _ bool) error

DropSeries drops the provided series from the index. If cascade is true and this is the last series to the measurement, the measurment will also be dropped.

func (*Index) EnableCompactions added in v1.5.0

func (i *Index) EnableCompactions()

func (*Index) FieldSet added in v1.5.0

func (i *Index) FieldSet() *tsdb.MeasurementFieldSet

FieldSet returns the assigned fieldset.

func (*Index) ForEachMeasurementName

func (i *Index) ForEachMeasurementName(fn func(name []byte) error) error

ForEachMeasurementName iterates over all measurement names in the index, applying fn. It returns the first error encountered, if any.

ForEachMeasurementName does not call fn on each partition concurrently so the call may provide a non-goroutine safe fn.

func (*Index) HasTagKey

func (i *Index) HasTagKey(name, key []byte) (bool, error)

HasTagKey returns true if tag key exists. It returns the first error encountered if any.

func (*Index) HasTagValue added in v1.5.0

func (i *Index) HasTagValue(name, key, value []byte) (bool, error)

HasTagValue returns true if tag value exists.

func (*Index) InitializeSeries

func (i *Index) InitializeSeries(keys, names [][]byte, tags []models.Tags) error

InitializeSeries is a no-op. This only applies to the in-memory index.

func (*Index) MeasurementExists

func (i *Index) MeasurementExists(name []byte) (bool, error)

MeasurementExists returns true if a measurement exists.

func (*Index) MeasurementHasSeries added in v1.5.0

func (i *Index) MeasurementHasSeries(name []byte) (bool, error)

MeasurementHasSeries returns true if a measurement has non-tombstoned series.

func (*Index) MeasurementIterator added in v1.5.0

func (i *Index) MeasurementIterator() (tsdb.MeasurementIterator, error)

MeasurementIterator returns an iterator over all measurements.

func (*Index) MeasurementNamesByRegex

func (i *Index) MeasurementNamesByRegex(re *regexp.Regexp) ([][]byte, error)

MeasurementNamesByRegex returns measurement names for the provided regex.

func (*Index) MeasurementSeriesIDIterator added in v1.5.0

func (i *Index) MeasurementSeriesIDIterator(name []byte) (tsdb.SeriesIDIterator, error)

MeasurementSeriesIDIterator returns an iterator over all series in a measurement.

func (*Index) MeasurementTagKeysByExpr

func (i *Index) MeasurementTagKeysByExpr(name []byte, expr influxql.Expr) (map[string]struct{}, error)

MeasurementTagKeysByExpr extracts the tag keys wanted by the expression.

func (*Index) MeasurementsSketches

func (i *Index) MeasurementsSketches() (estimator.Sketch, estimator.Sketch, error)

MeasurementsSketches returns the two measurement sketches for the index.

func (*Index) Open

func (i *Index) Open() error

Open opens the index.

func (*Index) PartitionAt added in v1.5.0

func (i *Index) PartitionAt(index int) *Partition

PartitionAt returns the partition by index.

func (*Index) Path

func (i *Index) Path() string

Path returns the path the index was opened with.

func (*Index) Rebuild added in v1.4.0

func (i *Index) Rebuild()

Rebuild rebuilds an index. It's a no-op for this index.

func (*Index) RetainFileSet

func (i *Index) RetainFileSet() (*FileSet, error)

RetainFileSet returns the set of all files across all partitions. This is only needed when all files need to be retained for an operation.

func (*Index) SeriesFile added in v1.5.0

func (i *Index) SeriesFile() *tsdb.SeriesFile

SeriesFile returns the series file attached to the index.

func (*Index) SeriesIDSet added in v1.5.0

func (i *Index) SeriesIDSet() *tsdb.SeriesIDSet

SeriesIDSet returns the set of series ids associated with series in this index. Any series IDs for series no longer present in the index are filtered out.

func (*Index) SeriesN

func (i *Index) SeriesN() int64

Since indexes are not shared across shards, the count returned by SeriesN cannot be combined with other shard's results. If you need to count series across indexes then use either the database-wide series file, or merge the index-level bitsets or sketches.

func (*Index) SeriesSketches

func (i *Index) SeriesSketches() (estimator.Sketch, estimator.Sketch, error)

SeriesSketches returns the two series sketches for the index.

func (*Index) SetFieldName

func (i *Index) SetFieldName(measurement []byte, name string)

SetFieldName is a no-op on this index.

func (*Index) SetFieldSet

func (i *Index) SetFieldSet(fs *tsdb.MeasurementFieldSet)

SetFieldSet sets a shared field set from the engine.

func (*Index) TagKeyCardinality

func (i *Index) TagKeyCardinality(name, key []byte) int

TagKeyCardinality always returns zero. It is not possible to determine cardinality of tags across index files, and thus it cannot be done across partitions.

func (*Index) TagKeyIterator added in v1.5.0

func (i *Index) TagKeyIterator(name []byte) (tsdb.TagKeyIterator, error)

TagKeyIterator returns an iterator for all keys across a single measurement.

func (*Index) TagKeySeriesIDIterator added in v1.5.0

func (i *Index) TagKeySeriesIDIterator(name, key []byte) (tsdb.SeriesIDIterator, error)

TagKeySeriesIDIterator returns a series iterator for all values across a single key.

func (*Index) TagValueIterator added in v1.5.0

func (i *Index) TagValueIterator(name, key []byte) (tsdb.TagValueIterator, error)

TagValueIterator returns an iterator for all values across a single key.

func (*Index) TagValueSeriesIDIterator added in v1.5.0

func (i *Index) TagValueSeriesIDIterator(name, key, value []byte) (tsdb.SeriesIDIterator, error)

TagValueSeriesIDIterator returns a series iterator for a single tag value.

func (*Index) Type

func (i *Index) Type() string

Type returns the type of Index this is.

func (*Index) UniqueReferenceID added in v1.6.0

func (i *Index) UniqueReferenceID() uintptr

func (*Index) Wait added in v1.3.8

func (i *Index) Wait()

Wait blocks until all outstanding compactions have completed.

func (*Index) WithLogger

func (i *Index) WithLogger(l *zap.Logger)

WithLogger sets the logger on the index after it's been created.

It's not safe to call WithLogger after the index has been opened, or before it has been closed.

type IndexFile

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

IndexFile represents a collection of measurement, tag, and series data.

func NewIndexFile

func NewIndexFile(sfile *tsdb.SeriesFile) *IndexFile

NewIndexFile returns a new instance of IndexFile.

func (*IndexFile) Close

func (f *IndexFile) Close() error

Close unmaps the data file.

func (*IndexFile) Compacting

func (f *IndexFile) Compacting() bool

Compacting returns true if the file is being compacted.

func (*IndexFile) HasSeries

func (f *IndexFile) HasSeries(name []byte, tags models.Tags, buf []byte) (exists, tombstoned bool)

HasSeries returns flags indicating if the series exists and if it is tombstoned.

func (*IndexFile) ID

func (f *IndexFile) ID() int

ID returns the file sequence identifier.

func (*IndexFile) Level

func (f *IndexFile) Level() int

Level returns the compaction level for the file.

func (*IndexFile) Measurement

func (f *IndexFile) Measurement(name []byte) MeasurementElem

Measurement returns a measurement element.

func (*IndexFile) MeasurementHasSeries added in v1.5.0

func (f *IndexFile) MeasurementHasSeries(ss *tsdb.SeriesIDSet, name []byte) (ok bool)

MeasurementHasSeries returns true if a measurement has any non-tombstoned series.

func (*IndexFile) MeasurementIterator

func (f *IndexFile) MeasurementIterator() MeasurementIterator

MeasurementIterator returns an iterator over all measurements.

func (*IndexFile) MeasurementN

func (f *IndexFile) MeasurementN() (n uint64)

MeasurementN returns the number of measurements in the file.

func (*IndexFile) MeasurementSeriesIDIterator added in v1.5.0

func (f *IndexFile) MeasurementSeriesIDIterator(name []byte) tsdb.SeriesIDIterator

MeasurementSeriesIDIterator returns an iterator over a measurement's series.

func (*IndexFile) MeasurementsSketches added in v1.6.3

func (f *IndexFile) MeasurementsSketches() (sketch, tSketch estimator.Sketch, err error)

MeasurementsSketches returns existence and tombstone sketches for measurements.

func (*IndexFile) Open

func (f *IndexFile) Open() error

Open memory maps the data file at the file's path.

func (*IndexFile) Path

func (f *IndexFile) Path() string

Path returns the file path.

func (*IndexFile) Release

func (f *IndexFile) Release()

Release removes a reference count from the file.

func (*IndexFile) Retain

func (f *IndexFile) Retain()

Retain adds a reference count to the file.

func (*IndexFile) SeriesIDSet added in v1.5.0

func (f *IndexFile) SeriesIDSet() (*tsdb.SeriesIDSet, error)

func (*IndexFile) SeriesSketches added in v1.6.3

func (f *IndexFile) SeriesSketches() (sketch, tSketch estimator.Sketch, err error)

SeriesSketches returns existence and tombstone sketches for series.

func (*IndexFile) SetPath

func (f *IndexFile) SetPath(path string)

SetPath sets the file's path.

func (*IndexFile) Size

func (f *IndexFile) Size() int64

Size returns the size of the index file, in bytes.

func (*IndexFile) TagKey

func (f *IndexFile) TagKey(name, key []byte) TagKeyElem

TagKey returns a tag key.

func (*IndexFile) TagKeyIterator

func (f *IndexFile) TagKeyIterator(name []byte) TagKeyIterator

TagKeyIterator returns an iterator over all tag keys for a measurement.

func (*IndexFile) TagKeySeriesIDIterator added in v1.5.0

func (f *IndexFile) TagKeySeriesIDIterator(name, key []byte) (tsdb.SeriesIDIterator, error)

TagKeySeriesIDIterator returns a series iterator for a tag key and a flag indicating if a tombstone exists on the measurement or key.

func (*IndexFile) TagValue

func (f *IndexFile) TagValue(name, key, value []byte) TagValueElem

TagValue returns a tag value.

func (*IndexFile) TagValueElem

func (f *IndexFile) TagValueElem(name, key, value []byte) TagValueElem

TagValueElem returns an element for a measurement/tag/value.

func (*IndexFile) TagValueIterator

func (f *IndexFile) TagValueIterator(name, key []byte) TagValueIterator

TagValueIterator returns a value iterator for a tag key and a flag indicating if a tombstone exists on the measurement or key.

func (*IndexFile) TagValueSeriesIDSet added in v1.7.0

func (f *IndexFile) TagValueSeriesIDSet(name, key, value []byte) (*tsdb.SeriesIDSet, error)

TagValueSeriesIDSet returns a series id set for a tag value.

func (*IndexFile) TombstoneSeriesIDSet added in v1.5.0

func (f *IndexFile) TombstoneSeriesIDSet() (*tsdb.SeriesIDSet, error)

func (*IndexFile) UnmarshalBinary

func (f *IndexFile) UnmarshalBinary(data []byte) error

UnmarshalBinary opens an index from data. The byte slice is retained so it must be kept open.

type IndexFileTrailer

type IndexFileTrailer struct {
	Version int

	MeasurementBlock struct {
		Offset int64
		Size   int64
	}

	SeriesIDSet struct {
		Offset int64
		Size   int64
	}

	TombstoneSeriesIDSet struct {
		Offset int64
		Size   int64
	}

	SeriesSketch struct {
		Offset int64
		Size   int64
	}

	TombstoneSeriesSketch struct {
		Offset int64
		Size   int64
	}
}

IndexFileTrailer represents meta data written to the end of the index file.

func ReadIndexFileTrailer

func ReadIndexFileTrailer(data []byte) (IndexFileTrailer, error)

ReadIndexFileTrailer returns the index file trailer from data.

func (*IndexFileTrailer) WriteTo

func (t *IndexFileTrailer) WriteTo(w io.Writer) (n int64, err error)

WriteTo writes the trailer to w.

type IndexFiles

type IndexFiles []*IndexFile

IndexFiles represents a layered set of index files.

func (IndexFiles) CompactTo

func (p IndexFiles) CompactTo(w io.Writer, sfile *tsdb.SeriesFile, m, k uint64, cancel <-chan struct{}) (n int64, err error)

CompactTo merges all index files and writes them to w.

func (IndexFiles) Files

func (p IndexFiles) Files() []File

Files returns p as a list of File objects.

func (IndexFiles) IDs

func (p IndexFiles) IDs() []int

IDs returns the ids for all index files.

func (IndexFiles) MeasurementIterator

func (p IndexFiles) MeasurementIterator() MeasurementIterator

MeasurementIterator returns an iterator that merges measurements across all files.

func (*IndexFiles) MeasurementNames

func (p *IndexFiles) MeasurementNames() [][]byte

MeasurementNames returns a sorted list of all measurement names for all files.

func (IndexFiles) MeasurementSeriesIDIterator added in v1.5.0

func (p IndexFiles) MeasurementSeriesIDIterator(name []byte) tsdb.SeriesIDIterator

MeasurementSeriesIDIterator returns an iterator that merges series across all files.

func (IndexFiles) Release

func (p IndexFiles) Release()

Release removes a reference count from all files.

func (IndexFiles) Retain

func (p IndexFiles) Retain()

Retain adds a reference count to all files.

func (IndexFiles) Stat

func (p IndexFiles) Stat() (*IndexFilesInfo, error)

Stat returns the max index file size and the total file size for all index files.

func (*IndexFiles) TagKeyIterator

func (p *IndexFiles) TagKeyIterator(name []byte) (TagKeyIterator, error)

TagKeyIterator returns an iterator that merges tag keys across all files.

func (IndexFiles) TagValueSeriesIDSet added in v1.7.0

func (p IndexFiles) TagValueSeriesIDSet(name, key, value []byte) (*tsdb.SeriesIDSet, error)

TagValueSeriesIDSet returns an iterator that merges series across all files.

type IndexFilesInfo

type IndexFilesInfo struct {
	MaxSize int64     // largest file size
	Size    int64     // total file size
	ModTime time.Time // last modified
}

type IndexOption added in v1.5.0

type IndexOption func(i *Index)

An IndexOption is a functional option for changing the configuration of an Index.

type LogEntry

type LogEntry struct {
	Flag     byte   // flag
	SeriesID uint64 // series id
	Name     []byte // measurement name
	Key      []byte // tag key
	Value    []byte // tag value
	Checksum uint32 // checksum of flag/name/tags.
	Size     int    // total size of record, in bytes.
	// contains filtered or unexported fields
}

LogEntry represents a single log entry in the write-ahead log.

func (*LogEntry) UnmarshalBinary

func (e *LogEntry) UnmarshalBinary(data []byte) error

UnmarshalBinary unmarshals data into e.

type LogFile

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

LogFile represents an on-disk write-ahead log file.

func NewLogFile

func NewLogFile(sfile *tsdb.SeriesFile, path string) *LogFile

NewLogFile returns a new instance of LogFile.

func (*LogFile) AddSeriesList

func (f *LogFile) AddSeriesList(seriesSet *tsdb.SeriesIDSet, names [][]byte, tagsSlice []models.Tags, tracker tsdb.StatsTracker) ([]uint64, error)

AddSeriesList adds a list of series to the log file in bulk.

func (*LogFile) Close

func (f *LogFile) Close() error

Close shuts down the file handle and mmap.

func (*LogFile) CompactTo

func (f *LogFile) CompactTo(w io.Writer, m, k uint64, cancel <-chan struct{}) (n int64, err error)

CompactTo compacts the log file and writes it to w.

func (*LogFile) DeleteMeasurement

func (f *LogFile) DeleteMeasurement(name []byte) error

DeleteMeasurement adds a tombstone for a measurement to the log file.

func (*LogFile) DeleteSeriesID added in v1.5.0

func (f *LogFile) DeleteSeriesID(id uint64) error

DeleteSeriesID adds a tombstone for a series id.

func (*LogFile) DeleteSeriesIDList added in v1.8.1

func (f *LogFile) DeleteSeriesIDList(ids []uint64) error

DeleteSeriesIDList adds a tombstone for seriesIDList

func (*LogFile) DeleteTagKey

func (f *LogFile) DeleteTagKey(name, key []byte) error

DeleteTagKey adds a tombstone for a tag key to the log file.

func (*LogFile) DeleteTagValue

func (f *LogFile) DeleteTagValue(name, key, value []byte) error

DeleteTagValue adds a tombstone for a tag value to the log file.

func (*LogFile) ExecEntries added in v1.8.3

func (f *LogFile) ExecEntries(entries []LogEntry) error

func (*LogFile) Filter

func (f *LogFile) Filter() *bloom.Filter

Filter returns the bloom filter for the file.

func (*LogFile) FlushAndSync added in v1.6.1

func (f *LogFile) FlushAndSync() error

FlushAndSync flushes buffered data to disk and then fsyncs the underlying file. If the LogFile has disabled flushing and syncing then FlushAndSync is a no-op.

func (*LogFile) ID

func (f *LogFile) ID() int

ID returns the file sequence identifier.

func (*LogFile) Level

func (f *LogFile) Level() int

Level returns the log level of the file.

func (*LogFile) Measurement

func (f *LogFile) Measurement(name []byte) MeasurementElem

Measurement returns a measurement element.

func (*LogFile) MeasurementHasSeries added in v1.5.0

func (f *LogFile) MeasurementHasSeries(ss *tsdb.SeriesIDSet, name []byte) bool

func (*LogFile) MeasurementIterator

func (f *LogFile) MeasurementIterator() MeasurementIterator

MeasurementIterator returns an iterator over all the measurements in the file.

func (*LogFile) MeasurementN

func (f *LogFile) MeasurementN() (n uint64)

MeasurementN returns the total number of measurements.

func (*LogFile) MeasurementNames

func (f *LogFile) MeasurementNames() []string

MeasurementNames returns an ordered list of measurement names.

func (*LogFile) MeasurementSeriesIDIterator added in v1.5.0

func (f *LogFile) MeasurementSeriesIDIterator(name []byte) tsdb.SeriesIDIterator

MeasurementSeriesIDIterator returns an iterator over all series for a measurement.

func (*LogFile) MeasurementsSketches added in v1.6.3

func (f *LogFile) MeasurementsSketches() (sketch, tSketch estimator.Sketch, err error)

MeasurementsSketches returns sketches for existing and tombstoned measurement names.

func (*LogFile) ModTime added in v1.9.4

func (f *LogFile) ModTime() time.Time

ModTime returns the last modified time of the file

func (*LogFile) Open

func (f *LogFile) Open() error

Open reads the log from a file and validates all the checksums.

func (*LogFile) Path

func (f *LogFile) Path() string

Path returns the file path.

func (*LogFile) Release

func (f *LogFile) Release()

Release removes a reference count from the file.

func (*LogFile) Retain

func (f *LogFile) Retain()

Retain adds a reference count to the file.

func (*LogFile) SeriesIDIterator added in v1.5.0

func (f *LogFile) SeriesIDIterator() tsdb.SeriesIDIterator

SeriesIDIterator returns an iterator over all series in the log file.

func (*LogFile) SeriesIDSet added in v1.5.0

func (f *LogFile) SeriesIDSet() (*tsdb.SeriesIDSet, error)

SeriesIDSet returns the series existence set.

func (*LogFile) SeriesN

func (f *LogFile) SeriesN() (n uint64)

SeriesN returns the total number of series in the file.

func (*LogFile) SeriesSketches added in v1.6.3

func (f *LogFile) SeriesSketches() (sketch, tSketch estimator.Sketch, err error)

SeriesSketches returns sketches for existing and tombstoned series.

func (*LogFile) SetPath

func (f *LogFile) SetPath(path string)

SetPath sets the log file's path.

func (*LogFile) Size

func (f *LogFile) Size() int64

Size returns the size of the file, in bytes.

func (*LogFile) Stat

func (f *LogFile) Stat() (int64, time.Time)

Stat returns size and last modification time of the file.

func (*LogFile) TagKey

func (f *LogFile) TagKey(name, key []byte) TagKeyElem

TagKey returns a tag key element.

func (*LogFile) TagKeyIterator

func (f *LogFile) TagKeyIterator(name []byte) TagKeyIterator

TagKeyIterator returns a value iterator for a measurement.

func (*LogFile) TagKeyN

func (f *LogFile) TagKeyN() (n uint64)

TagKeyN returns the total number of keys.

func (*LogFile) TagKeySeriesIDIterator added in v1.5.0

func (f *LogFile) TagKeySeriesIDIterator(name, key []byte) (tsdb.SeriesIDIterator, error)

TagKeySeriesIDIterator returns a series iterator for a tag key.

func (*LogFile) TagValue

func (f *LogFile) TagValue(name, key, value []byte) TagValueElem

TagValue returns a tag value element.

func (*LogFile) TagValueIterator

func (f *LogFile) TagValueIterator(name, key []byte) TagValueIterator

TagValueIterator returns a value iterator for a tag key.

func (*LogFile) TagValueN

func (f *LogFile) TagValueN() (n uint64)

TagValueN returns the total number of values.

func (*LogFile) TagValueSeriesIDSet added in v1.7.0

func (f *LogFile) TagValueSeriesIDSet(name, key, value []byte) (*tsdb.SeriesIDSet, error)

TagValueSeriesIDSet returns a series iterator for a tag value.

func (*LogFile) TombstoneSeriesIDSet added in v1.5.0

func (f *LogFile) TombstoneSeriesIDSet() (*tsdb.SeriesIDSet, error)

TombstoneSeriesIDSet returns the series tombstone set.

func (*LogFile) Writes added in v1.8.1

func (f *LogFile) Writes(entries []LogEntry) error

type Manifest

type Manifest struct {
	Levels []CompactionLevel `json:"levels,omitempty"`
	Files  []string          `json:"files,omitempty"`

	// Version should be updated whenever the TSI format has changed.
	Version int `json:"version,omitempty"`
	// contains filtered or unexported fields
}

Manifest represents the list of log & index files that make up the index. The files are listed in time order, not necessarily ID order.

func NewManifest

func NewManifest(path string) *Manifest

NewManifest returns a new instance of Manifest with default compaction levels.

func ReadManifestFile

func ReadManifestFile(path string) (*Manifest, int64, error)

ReadManifestFile reads a manifest from a file path and returns the Manifest, the size of the manifest on disk, and any error if appropriate.

func (*Manifest) HasFile

func (m *Manifest) HasFile(name string) bool

HasFile returns true if name is listed in the log files or index files.

func (*Manifest) Validate added in v1.4.0

func (m *Manifest) Validate() error

Validate checks if the Manifest's version is compatible with this version of the tsi1 index.

func (*Manifest) Write added in v1.5.0

func (m *Manifest) Write() (int64, error)

Write writes the manifest file to the provided path, returning the number of bytes written and an error, if any.

type MeasurementBlock

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

MeasurementBlock represents a collection of all measurements in an index.

func (*MeasurementBlock) Elem

func (blk *MeasurementBlock) Elem(name []byte) (e MeasurementBlockElem, ok bool)

Elem returns an element for a measurement.

func (*MeasurementBlock) Iterator

func (blk *MeasurementBlock) Iterator() MeasurementIterator

Iterator returns an iterator over all measurements.

func (*MeasurementBlock) SeriesIDIterator added in v1.5.0

func (blk *MeasurementBlock) SeriesIDIterator(name []byte) tsdb.SeriesIDIterator

SeriesIDIterator returns an iterator for all series ids in a measurement.

func (*MeasurementBlock) Sketches added in v1.6.3

func (blk *MeasurementBlock) Sketches() (sketch, tSketch estimator.Sketch, err error)

Sketches returns existence and tombstone measurement sketches.

func (*MeasurementBlock) UnmarshalBinary

func (blk *MeasurementBlock) UnmarshalBinary(data []byte) error

UnmarshalBinary unpacks data into the block. Block is not copied so data should be retained and unchanged after being passed into this function.

func (*MeasurementBlock) Version

func (blk *MeasurementBlock) Version() int

Version returns the encoding version parsed from the data. Only valid after UnmarshalBinary() has been successfully invoked.

type MeasurementBlockElem

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

MeasurementBlockElem represents an internal measurement element.

func (*MeasurementBlockElem) Deleted

func (e *MeasurementBlockElem) Deleted() bool

Deleted returns true if the tombstone flag is set.

func (*MeasurementBlockElem) ForEachSeriesID added in v1.5.0

func (e *MeasurementBlockElem) ForEachSeriesID(fn func(uint64) error) error

func (*MeasurementBlockElem) HasSeries added in v1.5.0

func (e *MeasurementBlockElem) HasSeries() bool

func (*MeasurementBlockElem) Name

func (e *MeasurementBlockElem) Name() []byte

Name returns the measurement name.

func (*MeasurementBlockElem) SeriesData

func (e *MeasurementBlockElem) SeriesData() []byte

SeriesData returns the raw series data.

func (*MeasurementBlockElem) SeriesID

func (e *MeasurementBlockElem) SeriesID(i int) uint64

SeriesID returns series ID at an index.

func (*MeasurementBlockElem) SeriesIDs

func (e *MeasurementBlockElem) SeriesIDs() []uint64

SeriesIDs returns a list of decoded series ids.

NOTE: This should be used for testing and diagnostics purposes only. It requires loading the entire list of series in-memory.

func (*MeasurementBlockElem) SeriesN

func (e *MeasurementBlockElem) SeriesN() uint64

SeriesN returns the number of series associated with the measurement.

func (*MeasurementBlockElem) Size

func (e *MeasurementBlockElem) Size() int

Size returns the size of the element.

func (*MeasurementBlockElem) TagBlockOffset

func (e *MeasurementBlockElem) TagBlockOffset() int64

TagBlockOffset returns the offset of the measurement's tag block.

func (*MeasurementBlockElem) TagBlockSize

func (e *MeasurementBlockElem) TagBlockSize() int64

TagBlockSize returns the size of the measurement's tag block.

func (*MeasurementBlockElem) UnmarshalBinary

func (e *MeasurementBlockElem) UnmarshalBinary(data []byte) error

UnmarshalBinary unmarshals data into e.

type MeasurementBlockTrailer

type MeasurementBlockTrailer struct {
	Version int // Encoding version

	// Offset & size of data section.
	Data struct {
		Offset int64
		Size   int64
	}

	// Offset & size of hash map section.
	HashIndex struct {
		Offset int64
		Size   int64
	}

	// Offset and size of cardinality sketch for measurements.
	Sketch struct {
		Offset int64
		Size   int64
	}

	// Offset and size of cardinality sketch for tombstoned measurements.
	TSketch struct {
		Offset int64
		Size   int64
	}
}

MeasurementBlockTrailer represents meta data at the end of a MeasurementBlock.

func ReadMeasurementBlockTrailer

func ReadMeasurementBlockTrailer(data []byte) (MeasurementBlockTrailer, error)

ReadMeasurementBlockTrailer returns the block trailer from data.

func (*MeasurementBlockTrailer) WriteTo

func (t *MeasurementBlockTrailer) WriteTo(w io.Writer) (n int64, err error)

WriteTo writes the trailer to w.

type MeasurementBlockWriter

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

MeasurementBlockWriter writes a measurement block.

func NewMeasurementBlockWriter

func NewMeasurementBlockWriter() *MeasurementBlockWriter

NewMeasurementBlockWriter returns a new MeasurementBlockWriter.

func (*MeasurementBlockWriter) Add

func (mw *MeasurementBlockWriter) Add(name []byte, deleted bool, offset, size int64, seriesIDs []uint64)

Add adds a measurement with series and tag set offset/size.

func (*MeasurementBlockWriter) WriteTo

func (mw *MeasurementBlockWriter) WriteTo(w io.Writer) (n int64, err error)

WriteTo encodes the measurements to w.

type MeasurementElem

type MeasurementElem interface {
	Name() []byte
	Deleted() bool
}

MeasurementElem represents a generic measurement element.

type MeasurementElems

type MeasurementElems []MeasurementElem

MeasurementElems represents a list of MeasurementElem.

func (MeasurementElems) Len

func (a MeasurementElems) Len() int

func (MeasurementElems) Less

func (a MeasurementElems) Less(i, j int) bool

func (MeasurementElems) Swap

func (a MeasurementElems) Swap(i, j int)

type MeasurementIterator

type MeasurementIterator interface {
	Next() MeasurementElem
}

MeasurementIterator represents a iterator over a list of measurements.

func MergeMeasurementIterators

func MergeMeasurementIterators(itrs ...MeasurementIterator) MeasurementIterator

MergeMeasurementIterators returns an iterator that merges a set of iterators. Iterators that are first in the list take precedence and a deletion by those early iterators will invalidate elements by later iterators.

type Partition added in v1.5.0

type Partition struct {

	// Log file compaction thresholds.
	MaxLogFileSize int64
	MaxLogFileAge  time.Duration
	// contains filtered or unexported fields
}

Partition represents a collection of layered index files and WAL.

func NewPartition added in v1.5.0

func NewPartition(sfile *tsdb.SeriesFile, path string) *Partition

NewPartition returns a new instance of Partition.

func (*Partition) AssignShard added in v1.5.0

func (p *Partition) AssignShard(k string, shardID uint64)

func (*Partition) CheckLogFile added in v1.5.0

func (p *Partition) CheckLogFile() error

func (*Partition) Close added in v1.5.0

func (p *Partition) Close() error

Close closes the index.

func (*Partition) Compact added in v1.5.0

func (p *Partition) Compact()

Compact requests a compaction of log files.

func (*Partition) CurrentCompactionN added in v1.7.9

func (p *Partition) CurrentCompactionN() int

CurrentCompactionN returns the number of compactions currently running.

func (*Partition) DisableCompactions added in v1.5.0

func (p *Partition) DisableCompactions()

func (*Partition) DropMeasurement added in v1.5.0

func (p *Partition) DropMeasurement(name []byte) error

DropMeasurement deletes a measurement from the index. DropMeasurement does not remove any series from the index directly.

func (*Partition) DropSeries added in v1.5.0

func (p *Partition) DropSeries(seriesID uint64) error

func (*Partition) DropSeriesList added in v1.8.1

func (p *Partition) DropSeriesList(seriesIDs []uint64) error

func (*Partition) EnableCompactions added in v1.5.0

func (p *Partition) EnableCompactions()

func (*Partition) FieldSet added in v1.5.0

func (p *Partition) FieldSet() *tsdb.MeasurementFieldSet

FieldSet returns the fieldset.

func (*Partition) FileN added in v1.5.0

func (p *Partition) FileN() int

FileN returns the active files in the file set.

func (*Partition) ForEachMeasurementName added in v1.5.0

func (p *Partition) ForEachMeasurementName(fn func(name []byte) error) error

ForEachMeasurementName iterates over all measurement names in the index.

func (*Partition) ForEachMeasurementTagKey added in v1.5.0

func (p *Partition) ForEachMeasurementTagKey(name []byte, fn func(key []byte) error) error

ForEachMeasurementTagKey iterates over all tag keys in a measurement.

func (*Partition) HasTagKey added in v1.5.0

func (p *Partition) HasTagKey(name, key []byte) (bool, error)

HasTagKey returns true if tag key exists.

func (*Partition) HasTagValue added in v1.5.0

func (p *Partition) HasTagValue(name, key, value []byte) (bool, error)

HasTagValue returns true if tag value exists.

func (*Partition) Manifest added in v1.5.0

func (p *Partition) Manifest() *Manifest

Manifest returns a manifest for the index.

func (*Partition) ManifestPath added in v1.5.0

func (p *Partition) ManifestPath() string

ManifestPath returns the path to the index's manifest file.

func (*Partition) MeasurementExists added in v1.5.0

func (p *Partition) MeasurementExists(name []byte) (bool, error)

MeasurementExists returns true if a measurement exists.

func (*Partition) MeasurementHasSeries added in v1.5.0

func (p *Partition) MeasurementHasSeries(name []byte) (bool, error)

MeasurementHasSeries returns true if a measurement has at least one non-tombstoned series.

func (*Partition) MeasurementIterator added in v1.5.0

func (p *Partition) MeasurementIterator() (tsdb.MeasurementIterator, error)

MeasurementIterator returns an iterator over all measurement names.

func (*Partition) MeasurementNamesByRegex added in v1.5.0

func (p *Partition) MeasurementNamesByRegex(re *regexp.Regexp) ([][]byte, error)

func (*Partition) MeasurementSeriesIDIterator added in v1.5.0

func (p *Partition) MeasurementSeriesIDIterator(name []byte) (tsdb.SeriesIDIterator, error)

func (*Partition) MeasurementTagKeysByExpr added in v1.5.0

func (p *Partition) MeasurementTagKeysByExpr(name []byte, expr influxql.Expr) (map[string]struct{}, error)

MeasurementTagKeysByExpr extracts the tag keys wanted by the expression.

func (*Partition) MeasurementsSketches added in v1.5.0

func (p *Partition) MeasurementsSketches() (estimator.Sketch, estimator.Sketch, error)

MeasurementsSketches returns the two sketches for the partition by merging all instances of the type sketch types in all the index files.

func (*Partition) NeedsCompaction added in v1.9.4

func (p *Partition) NeedsCompaction() bool

needsCompaction only requires a read lock and checks if there are files that could be compacted. If compact is updated we should also update needsCompaction

func (*Partition) NextSequence added in v1.5.0

func (p *Partition) NextSequence() int

NextSequence returns the next file identifier.

func (*Partition) Open added in v1.5.0

func (p *Partition) Open() error

Open opens the partition.

func (*Partition) Path added in v1.5.0

func (p *Partition) Path() string

Path returns the path to the partition.

func (*Partition) Rebuild added in v1.5.0

func (p *Partition) Rebuild()

func (*Partition) RemoveShard added in v1.5.0

func (p *Partition) RemoveShard(shardID uint64)

func (*Partition) RetainFileSet added in v1.5.0

func (p *Partition) RetainFileSet() (*FileSet, error)

RetainFileSet returns the current fileset and adds a reference count.

func (*Partition) SeriesFile added in v1.5.0

func (p *Partition) SeriesFile() *tsdb.SeriesFile

SeriesFile returns the attached series file.

func (*Partition) SeriesSketches added in v1.5.0

func (p *Partition) SeriesSketches() (estimator.Sketch, estimator.Sketch, error)

SeriesSketches returns the two sketches for the partition by merging all instances of the type sketch types in all the index files.

func (*Partition) SetFieldName added in v1.5.0

func (p *Partition) SetFieldName(measurement []byte, name string)

func (*Partition) SetFieldSet added in v1.5.0

func (p *Partition) SetFieldSet(fs *tsdb.MeasurementFieldSet)

SetFieldSet sets a shared field set from the engine.

func (*Partition) TagKeyCardinality added in v1.5.0

func (p *Partition) TagKeyCardinality(name, key []byte) int

TagKeyCardinality always returns zero. It is not possible to determine cardinality of tags across index files.

func (*Partition) TagKeyIterator added in v1.5.0

func (p *Partition) TagKeyIterator(name []byte) tsdb.TagKeyIterator

TagKeyIterator returns an iterator for all keys across a single measurement.

func (*Partition) TagKeySeriesIDIterator added in v1.5.0

func (p *Partition) TagKeySeriesIDIterator(name, key []byte) (tsdb.SeriesIDIterator, error)

TagKeySeriesIDIterator returns a series iterator for all values across a single key.

func (*Partition) TagValueIterator added in v1.5.0

func (p *Partition) TagValueIterator(name, key []byte) tsdb.TagValueIterator

TagValueIterator returns an iterator for all values across a single key.

func (*Partition) TagValueSeriesIDIterator added in v1.5.0

func (p *Partition) TagValueSeriesIDIterator(name, key, value []byte) (tsdb.SeriesIDIterator, error)

TagValueSeriesIDIterator returns a series iterator for a single key value.

func (*Partition) Wait added in v1.5.0

func (p *Partition) Wait()

Wait will block until all compactions are finished. Must only be called while they are disabled.

func (*Partition) WithLogger added in v1.5.0

func (p *Partition) WithLogger(logger *zap.Logger)

WithLogger sets the logger for the index.

type TagBlock

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

TagBlock represents tag key/value block for a single measurement.

func (*TagBlock) DecodeTagKeyElem added in v1.5.0

func (blk *TagBlock) DecodeTagKeyElem(key []byte, elem *TagBlockKeyElem) bool

func (*TagBlock) DecodeTagValueElem added in v1.5.0

func (blk *TagBlock) DecodeTagValueElem(key, value []byte, valueElem *TagBlockValueElem) bool

DecodeTagValueElem returns an element for a tag value.

func (*TagBlock) TagKeyElem

func (blk *TagBlock) TagKeyElem(key []byte) TagKeyElem

TagKeyElem returns an element for a tag key. Returns an element with a nil key if not found.

func (*TagBlock) TagKeyIterator

func (blk *TagBlock) TagKeyIterator() TagKeyIterator

TagKeyIterator returns an iterator over all the keys in the block.

func (*TagBlock) TagValueElem

func (blk *TagBlock) TagValueElem(key, value []byte) TagValueElem

TagValueElem returns an element for a tag value.

func (*TagBlock) UnmarshalBinary

func (blk *TagBlock) UnmarshalBinary(data []byte) error

UnmarshalBinary unpacks data into the tag block. Tag block is not copied so data should be retained and unchanged after being passed into this function.

func (*TagBlock) Version

func (blk *TagBlock) Version() int

Version returns the encoding version parsed from the data. Only valid after UnmarshalBinary() has been successfully invoked.

type TagBlockEncoder

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

TagBlockEncoder encodes a tags to a TagBlock section.

func NewTagBlockEncoder

func NewTagBlockEncoder(w io.Writer) *TagBlockEncoder

NewTagBlockEncoder returns a new TagBlockEncoder.

func (*TagBlockEncoder) Close

func (enc *TagBlockEncoder) Close() error

Close flushes the trailer of the encoder to the writer.

func (*TagBlockEncoder) EncodeKey

func (enc *TagBlockEncoder) EncodeKey(key []byte, deleted bool) error

EncodeKey writes a tag key to the underlying writer.

func (*TagBlockEncoder) EncodeValue

func (enc *TagBlockEncoder) EncodeValue(value []byte, deleted bool, ss *tsdb.SeriesIDSet) error

EncodeValue writes a tag value to the underlying writer. The tag key must be lexicographical sorted after the previous encoded tag key.

func (*TagBlockEncoder) N

func (enc *TagBlockEncoder) N() int64

N returns the number of bytes written.

type TagBlockKeyElem

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

TagBlockKeyElem represents a tag key element in a TagBlock.

func (*TagBlockKeyElem) Deleted

func (e *TagBlockKeyElem) Deleted() bool

Deleted returns true if the key has been tombstoned.

func (*TagBlockKeyElem) Key

func (e *TagBlockKeyElem) Key() []byte

Key returns the key name of the element.

func (*TagBlockKeyElem) TagValueIterator

func (e *TagBlockKeyElem) TagValueIterator() TagValueIterator

TagValueIterator returns an iterator over the key's values.

type TagBlockTrailer

type TagBlockTrailer struct {
	Version int   // Encoding version
	Size    int64 // Total size w/ trailer

	// Offset & size of value data section.
	ValueData struct {
		Offset int64
		Size   int64
	}

	// Offset & size of key data section.
	KeyData struct {
		Offset int64
		Size   int64
	}

	// Offset & size of hash map section.
	HashIndex struct {
		Offset int64
		Size   int64
	}
}

TagBlockTrailer represents meta data at the end of a TagBlock.

func ReadTagBlockTrailer

func ReadTagBlockTrailer(data []byte) (TagBlockTrailer, error)

ReadTagBlockTrailer returns the tag block trailer from data.

func (*TagBlockTrailer) WriteTo

func (t *TagBlockTrailer) WriteTo(w io.Writer) (n int64, err error)

WriteTo writes the trailer to w.

type TagBlockValueElem

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

TagBlockValueElem represents a tag value element.

func (*TagBlockValueElem) Deleted

func (e *TagBlockValueElem) Deleted() bool

Deleted returns true if the element has been tombstoned.

func (*TagBlockValueElem) SeriesData

func (e *TagBlockValueElem) SeriesData() []byte

SeriesData returns the raw series data.

func (*TagBlockValueElem) SeriesID

func (e *TagBlockValueElem) SeriesID(i int) uint64

SeriesID returns series ID at an index.

func (*TagBlockValueElem) SeriesIDSet added in v1.7.0

func (e *TagBlockValueElem) SeriesIDSet() (*tsdb.SeriesIDSet, error)

SeriesIDSet returns a set of series ids.

func (*TagBlockValueElem) SeriesIDs

func (e *TagBlockValueElem) SeriesIDs() ([]uint64, error)

SeriesIDs returns a list decoded series ids.

func (*TagBlockValueElem) SeriesN

func (e *TagBlockValueElem) SeriesN() uint64

SeriesN returns the series count.

func (*TagBlockValueElem) Size

func (e *TagBlockValueElem) Size() int

Size returns the size of the element.

func (*TagBlockValueElem) Value

func (e *TagBlockValueElem) Value() []byte

Value returns the value for the element.

type TagKeyElem

type TagKeyElem interface {
	Key() []byte
	Deleted() bool
	TagValueIterator() TagValueIterator
}

TagKeyElem represents a generic tag key element.

type TagKeyIterator

type TagKeyIterator interface {
	Next() TagKeyElem
}

TagKeyIterator represents a iterator over a list of tag keys.

func MergeTagKeyIterators

func MergeTagKeyIterators(itrs ...TagKeyIterator) TagKeyIterator

MergeTagKeyIterators returns an iterator that merges a set of iterators. Iterators that are first in the list take precedence and a deletion by those early iterators will invalidate elements by later iterators.

type TagValueElem

type TagValueElem interface {
	Value() []byte
	Deleted() bool
}

TagValueElem represents a generic tag value element.

type TagValueIterator

type TagValueIterator interface {
	Next() TagValueElem
}

TagValueIterator represents a iterator over a list of tag values.

func MergeTagValueIterators

func MergeTagValueIterators(itrs ...TagValueIterator) TagValueIterator

MergeTagValueIterators returns an iterator that merges a set of iterators. Iterators that are first in the list take precedence and a deletion by those early iterators will invalidate elements by later iterators.

type TagValueSeriesIDCache added in v1.7.0

type TagValueSeriesIDCache struct {
	sync.RWMutex
	// contains filtered or unexported fields
}

TagValueSeriesIDCache is an LRU cache for series id sets associated with name -> key -> value mappings. The purpose of the cache is to provide efficient means to get sets of series ids that would otherwise involve merging many individual bitmaps at query time.

When initialising a TagValueSeriesIDCache a capacity must be provided. When more than c items are added to the cache, the least recently used item is evicted from the cache.

A TagValueSeriesIDCache comprises a linked list implementation to track the order by which items should be evicted from the cache, and a hashmap implementation to provide constant time retrievals of items from the cache.

func NewTagValueSeriesIDCache added in v1.7.0

func NewTagValueSeriesIDCache(c int) *TagValueSeriesIDCache

NewTagValueSeriesIDCache returns a TagValueSeriesIDCache with capacity c.

func (*TagValueSeriesIDCache) Delete added in v1.7.0

func (c *TagValueSeriesIDCache) Delete(name, key, value []byte, x uint64)

Delete removes x from the tuple {name, key, value} if it exists. This method takes a lock on the underlying SeriesIDSet.

func (*TagValueSeriesIDCache) Get added in v1.7.0

func (c *TagValueSeriesIDCache) Get(name, key, value []byte) *tsdb.SeriesIDSet

Get returns the SeriesIDSet associated with the {name, key, value} tuple if it exists.

func (*TagValueSeriesIDCache) Put added in v1.7.0

func (c *TagValueSeriesIDCache) Put(name, key, value []byte, ss *tsdb.SeriesIDSet)

Put adds the SeriesIDSet to the cache under the tuple {name, key, value}. If the cache is at its limit, then the least recently used item is evicted.

Jump to

Keyboard shortcuts

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