types

package
v0.0.0-...-e32b5e2 Latest Latest
Warning

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

Go to latest
Published: Nov 11, 2024 License: AGPL-3.0 Imports: 12 Imported by: 0

Documentation

Index

Constants

View Source
const (
	FPointSize           = uint64(unsafe.Sizeof(promql.FPoint{}))
	HPointSize           = uint64(FPointSize * nativeHistogramSampleSizeFactor)
	VectorSampleSize     = uint64(unsafe.Sizeof(promql.Sample{})) // This assumes each sample is a float sample, not a histogram.
	Float64Size          = uint64(unsafe.Sizeof(float64(0)))
	BoolSize             = uint64(unsafe.Sizeof(false))
	HistogramPointerSize = uint64(unsafe.Sizeof((*histogram.FloatHistogram)(nil)))
)

Variables

View Source
var (
	FPointSlicePool = NewLimitingBucketedPool(
		pool.NewBucketedPool(1, maxExpectedPointsPerSeries, pointsPerSeriesBucketFactor, func(size int) []promql.FPoint {
			return make([]promql.FPoint, 0, size)
		}),
		FPointSize,
		false,
	)

	HPointSlicePool = NewLimitingBucketedPool(
		pool.NewBucketedPool(1, maxExpectedPointsPerSeries, pointsPerSeriesBucketFactor, func(size int) []promql.HPoint {
			return make([]promql.HPoint, 0, size)
		}),
		HPointSize,
		false,
	)

	VectorPool = NewLimitingBucketedPool(
		pool.NewBucketedPool(1, maxExpectedPointsPerSeries, pointsPerSeriesBucketFactor, func(size int) promql.Vector {
			return make(promql.Vector, 0, size)
		}),
		VectorSampleSize,
		false,
	)

	Float64SlicePool = NewLimitingBucketedPool(
		pool.NewBucketedPool(1, maxExpectedPointsPerSeries, pointsPerSeriesBucketFactor, func(size int) []float64 {
			return make([]float64, 0, size)
		}),
		Float64Size,
		true,
	)

	BoolSlicePool = NewLimitingBucketedPool(
		pool.NewBucketedPool(1, maxExpectedPointsPerSeries, pointsPerSeriesBucketFactor, func(size int) []bool {
			return make([]bool, 0, size)
		}),
		BoolSize,
		true,
	)

	HistogramSlicePool = NewLimitingBucketedPool(
		pool.NewBucketedPool(1, maxExpectedPointsPerSeries, pointsPerSeriesBucketFactor, func(size int) []*histogram.FloatHistogram {
			return make([]*histogram.FloatHistogram, 0, size)
		}),
		HistogramPointerSize,
		true,
	)
)
View Source
var EOS = errors.New("operator stream exhausted") //nolint:revive

Functions

func GetMatrix

func GetMatrix(size int) promql.Matrix

func HasDuplicateSeries

func HasDuplicateSeries(metadata []SeriesMetadata) bool

func PutInstantVectorSeriesData

func PutInstantVectorSeriesData(d InstantVectorSeriesData, tracker *limiting.MemoryConsumptionTracker)

PutInstantVectorSeriesData is equivalent to calling FPointSlicePool.Put(d.Floats) and HPointSlicePool.Put(d.Histograms).

func PutMatrix

func PutMatrix(m promql.Matrix)

func PutSeriesMetadataSlice

func PutSeriesMetadataSlice(s []SeriesMetadata)

Types

type AnnotationGenerator

type AnnotationGenerator func(metricName string, expressionPosition posrange.PositionRange) error

AnnotationGenerator is a function that returns an annotation for the given metric name and expression position.

type EmitAnnotationFunc

type EmitAnnotationFunc func(generator AnnotationGenerator)

EmitAnnotationFunc is a function that emits the annotation created by generator.

type FPointRingBuffer

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

func NewFPointRingBuffer

func NewFPointRingBuffer(memoryConsumptionTracker *limiting.MemoryConsumptionTracker) *FPointRingBuffer

func (*FPointRingBuffer) Append

func (b *FPointRingBuffer) Append(p promql.FPoint) error

Append adds p to this buffer, expanding it if required. If this buffer is non-empty, p.T must be greater than or equal to the timestamp of the last point in the buffer.

func (*FPointRingBuffer) Close

func (b *FPointRingBuffer) Close()

Close releases any resources associated with this buffer.

func (*FPointRingBuffer) DiscardPointsBefore

func (b *FPointRingBuffer) DiscardPointsBefore(t int64)

DiscardPointsBefore discards all points in this buffer with timestamp less than t.

func (*FPointRingBuffer) Release

func (b *FPointRingBuffer) Release()

Release clears the contents of this buffer and releases the underlying point slice. The buffer can be used again and will acquire a new slice when required.

func (*FPointRingBuffer) Reset

func (b *FPointRingBuffer) Reset()

Reset clears the contents of this buffer, but retains the underlying point slice for future reuse.

func (*FPointRingBuffer) Use

func (b *FPointRingBuffer) Use(s []promql.FPoint)

Use replaces the contents of this buffer with s. The points in s must be in time order, not contain duplicate timestamps and start at index 0. s will be modified in place when the buffer is modified, and callers should not modify s after passing it off to the ring buffer via Use. s will be returned to the pool when Close is called, Use is called again, or the buffer needs to expand, so callers should not return s to the pool themselves. s must have a capacity that is a power of two.

func (*FPointRingBuffer) ViewUntilSearchingBackwards

func (b *FPointRingBuffer) ViewUntilSearchingBackwards(maxT int64, existing *FPointRingBufferView) *FPointRingBufferView

ViewUntilSearchingBackwards is like ViewUntilSearchingForwards, except it examines the points from the end of the buffer, so is preferred over ViewUntilSearchingForwards if it is expected that only a few of the points will have timestamp greater than maxT.

func (*FPointRingBuffer) ViewUntilSearchingForwards

func (b *FPointRingBuffer) ViewUntilSearchingForwards(maxT int64, existing *FPointRingBufferView) *FPointRingBufferView

ViewUntilSearchingForwards returns a view into this buffer, including only points with timestamps less than or equal to maxT. ViewUntilSearchingForwards examines the points in the buffer starting from the front of the buffer, so is preferred over ViewUntilSearchingBackwards if it is expected that there are many points with timestamp greater than maxT, and few points with earlier timestamps. existing is an existing view instance for this buffer that is reused if provided. It can be nil. The returned view is no longer valid if this buffer is modified (eg. a point is added, or the buffer is reset or closed).

type FPointRingBufferView

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

func (FPointRingBufferView) Any

func (v FPointRingBufferView) Any() bool

Any returns true if this ring buffer view contains any points.

func (FPointRingBufferView) CopyPoints

func (v FPointRingBufferView) CopyPoints() ([]promql.FPoint, error)

CopyPoints returns a single slice of the points in this buffer view. Callers may modify the values in the returned slice, and should return the slice to the pool by calling PutFPointSlice when it is no longer needed. Calling UnsafePoints is more efficient than calling CopyPoints, as CopyPoints will create a new slice and copy all points into the slice, whereas UnsafePoints returns a view into the internal state of this buffer.

func (FPointRingBufferView) Count

func (v FPointRingBufferView) Count() int

Count returns the number of points in this ring buffer view.

func (FPointRingBufferView) First

First returns the first point in this ring buffer view. It panics if the buffer is empty.

func (FPointRingBufferView) ForEach

func (v FPointRingBufferView) ForEach(f func(p promql.FPoint))

ForEach calls f for each point in this buffer view.

func (FPointRingBufferView) Last

func (v FPointRingBufferView) Last() (promql.FPoint, bool)

Last returns the last point in this ring buffer view. It returns false if the view is empty.

func (FPointRingBufferView) UnsafePoints

func (v FPointRingBufferView) UnsafePoints() (head []promql.FPoint, tail []promql.FPoint)

UnsafePoints returns slices of the points in this buffer view. Either or both slice could be empty. Callers must not modify the values in the returned slices nor return them to a pool. Calling UnsafePoints is more efficient than calling CopyPoints, as CopyPoints will create a new slice and copy all points into the slice, whereas UnsafePoints returns a view into the internal state of the buffer. The returned slices are no longer valid if this buffer is modified (eg. a point is added, or the buffer is reset or closed).

FIXME: the fact we have to expose this is a bit gross, but the overhead of calling a function with ForEach is terrible. Perhaps we can use range-over function iterators (https://go.dev/wiki/RangefuncExperiment) once this is not experimental?

type HPointRingBuffer

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

func NewHPointRingBuffer

func NewHPointRingBuffer(memoryConsumptionTracker *limiting.MemoryConsumptionTracker) *HPointRingBuffer

func (*HPointRingBuffer) Append

func (b *HPointRingBuffer) Append(p promql.HPoint) error

Append adds p to this buffer, expanding it if required. If this buffer is non-empty, p.T must be greater than or equal to the timestamp of the last point in the buffer.

func (*HPointRingBuffer) Close

func (b *HPointRingBuffer) Close()

Close releases any resources associated with this buffer.

func (*HPointRingBuffer) DiscardPointsBefore

func (b *HPointRingBuffer) DiscardPointsBefore(t int64)

DiscardPointsBefore discards all points in this buffer with timestamp less than t.

func (*HPointRingBuffer) NextPoint

func (b *HPointRingBuffer) NextPoint() (*promql.HPoint, error)

NextPoint gets the next point in this buffer, expanding it if required. The returned point's timestamp (HPoint.T) must be set to greater than or equal to the timestamp of the last point in the buffer before further methods are called on this buffer (with the exception of RemoveLastPoint, Reset or Close).

This method allows reusing an existing HPoint in this buffer where possible, reducing the number of FloatHistograms allocated.

func (*HPointRingBuffer) Release

func (b *HPointRingBuffer) Release()

Release clears the contents of this buffer and releases the underlying point slice. The buffer can be used again and will acquire a new slice when required.

func (*HPointRingBuffer) RemoveLastPoint

func (b *HPointRingBuffer) RemoveLastPoint()

RemoveLastPoint removes the last point that was allocated. This is used for when NextPoint allocates a point that is then unused and needs to be returned to the ring buffer. This occurs when a histogram point has a stale marker. It panics if the buffer is empty.

func (*HPointRingBuffer) Reset

func (b *HPointRingBuffer) Reset()

Reset clears the contents of this buffer, but retains the underlying point slice for future reuse.

func (*HPointRingBuffer) Use

func (b *HPointRingBuffer) Use(s []promql.HPoint)

Use replaces the contents of this buffer with s. The points in s must be in time order, not contain duplicate timestamps and start at index 0. s will be modified in place when the buffer is modified, and callers should not modify s after passing it off to the ring buffer via Use. s will be returned to the pool when Close is called, Use is called again, or the buffer needs to expand, so callers should not return s to the pool themselves. s must have a capacity that is a power of two.

func (*HPointRingBuffer) ViewUntilSearchingBackwards

func (b *HPointRingBuffer) ViewUntilSearchingBackwards(maxT int64, existing *HPointRingBufferView) *HPointRingBufferView

ViewUntilSearchingBackwards is like ViewUntilSearchingForwards, except it examines the points from the end of the buffer, so is preferred over ViewUntilSearchingForwards if it is expected that only a few of the points will have timestamp greater than maxT.

func (*HPointRingBuffer) ViewUntilSearchingForwards

func (b *HPointRingBuffer) ViewUntilSearchingForwards(maxT int64, existing *HPointRingBufferView) *HPointRingBufferView

ViewUntilSearchingForwards returns a view into this buffer, including only points with timestamps less than or equal to maxT. ViewUntilSearchingForwards examines the points in the buffer starting from the front of the buffer, so is preferred over ViewUntilSearchingBackwards if it is expected that there are many points with timestamp greater than maxT, and few points with earlier timestamps. existing is an existing view instance for this buffer that is reused if provided. It can be nil. The returned view is no longer valid if this buffer is modified (eg. a point is added, or the buffer is reset or closed).

type HPointRingBufferView

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

func (HPointRingBufferView) Any

func (v HPointRingBufferView) Any() bool

Any returns true if this ring buffer view contains any points.

func (HPointRingBufferView) CopyPoints

func (v HPointRingBufferView) CopyPoints() ([]promql.HPoint, error)

CopyPoints returns a single slice of the points in this buffer view. Callers may modify the values in the returned slice, and should return the slice to the pool by calling PutHPointSlice when it is no longer needed. Calling UnsafePoints is more efficient than calling CopyPoints, as CopyPoints will create a new slice and copy all points into the slice, whereas UnsafePoints returns a view into the internal state of this buffer.

func (HPointRingBufferView) Count

func (v HPointRingBufferView) Count() int

Count returns the number of points in this ring buffer view.

func (HPointRingBufferView) First

First returns the first point in this ring buffer view. It panics if the buffer is empty.

func (HPointRingBufferView) ForEach

func (v HPointRingBufferView) ForEach(f func(p promql.HPoint))

ForEach calls f for each point in this buffer view.

func (HPointRingBufferView) Last

func (v HPointRingBufferView) Last() (promql.HPoint, bool)

Last returns the last point in this ring buffer view. It returns false if the view is empty.

func (HPointRingBufferView) UnsafePoints

func (v HPointRingBufferView) UnsafePoints() (head []promql.HPoint, tail []promql.HPoint)

UnsafePoints returns slices of the points in this buffer view. Either or both slice could be empty. Callers must not modify the values in the returned slices nor return them to a pool. Calling UnsafePoints is more efficient than calling CopyPoints, as CopyPoints will create a new slice and copy all points into the slice, whereas UnsafePoints returns a view into the internal state of the buffer. The returned slices are no longer valid if this buffer is modified (eg. a point is added, or the buffer is reset or closed).

FIXME: the fact we have to expose this is a bit gross, but the overhead of calling a function with ForEach is terrible. Perhaps we can use range-over function iterators (https://go.dev/wiki/RangefuncExperiment) once this is not experimental?

type InstantVectorOperator

type InstantVectorOperator interface {
	SeriesOperator

	// NextSeries returns the next series from this operator, or EOS if no more series are available.
	// SeriesMetadata must be called exactly once before calling NextSeries.
	// The returned InstantVectorSeriesData can be modified by the caller or returned to a pool.
	// The returned InstantVectorSeriesData can contain no points.
	NextSeries(ctx context.Context) (InstantVectorSeriesData, error)
}

InstantVectorOperator represents all operators that produce instant vectors.

type InstantVectorSeriesData

type InstantVectorSeriesData struct {
	// Floats contains floating point samples for this series.
	// Samples must be sorted in timestamp order, earliest timestamps first.
	// Samples must not have duplicate timestamps.
	Floats []promql.FPoint

	// Histograms contains histogram samples for this series.
	// Samples must be sorted in timestamp order, earliest timestamps first.
	// Samples must not have duplicate timestamps.
	// Samples must not share FloatHistogram instances.
	Histograms []promql.HPoint
}

type InstantVectorSeriesDataIterator

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

func (*InstantVectorSeriesDataIterator) Next

Next returns either a float or histogram iterating through both sets of points. It returns the next point with the lowest timestamp. If h is not nil, the value is a histogram, otherwise it is a float. If no more values exist ok is false.

func (*InstantVectorSeriesDataIterator) Reset

type LimitingBucketedPool

type LimitingBucketedPool[S ~[]E, E any] struct {
	// contains filtered or unexported fields
}

LimitingBucketedPool pools slices across multiple query evaluations, and applies any max in-memory bytes limit.

LimitingBucketedPool only estimates the in-memory size of the slices it returns. For example, it ignores the overhead of slice headers, assumes all native histograms are the same size, and assumes all elements of a promql.Vector are float samples.

func NewLimitingBucketedPool

func NewLimitingBucketedPool[S ~[]E, E any](inner *pool.BucketedPool[S, E], elementSize uint64, clearOnGet bool) *LimitingBucketedPool[S, E]

func (*LimitingBucketedPool[S, E]) Get

func (p *LimitingBucketedPool[S, E]) Get(size int, tracker *limiting.MemoryConsumptionTracker) (S, error)

Get returns a slice of E of length 0 and capacity greater than or equal to size.

If the capacity of the returned slice would cause the max memory consumption limit to be exceeded, then an error is returned.

Note that the capacity of the returned slice may be significantly larger than size, depending on the configuration of the underlying bucketed pool.

func (*LimitingBucketedPool[S, E]) Put

func (p *LimitingBucketedPool[S, E]) Put(s S, tracker *limiting.MemoryConsumptionTracker)

Put returns a slice of E to the pool and updates the current memory consumption.

type Operator

type Operator interface {
	// ExpressionPosition returns the position of the PromQL expression that this operator represents.
	ExpressionPosition() posrange.PositionRange

	// Close frees all resources associated with this operator.
	// Calling SeriesMetadata or NextSeries after calling Close may result in unpredictable behaviour, corruption or crashes.
	// It must be safe to call Close at any time, including if SeriesMetadata or NextSeries have returned an error.
	Close()
}

Operator represents all operators.

type QueryTimeRange

type QueryTimeRange struct {
	StartT               int64 // Start timestamp, in milliseconds since Unix epoch.
	EndT                 int64 // End timestamp, in milliseconds since Unix epoch.
	IntervalMilliseconds int64 // Range query interval, or 1 for instant queries. Note that this is deliberately different to parser.EvalStmt.Interval for instant queries (where it is 0) to simplify some loop conditions.

	StepCount int // 1 for instant queries.
}

func NewInstantQueryTimeRange

func NewInstantQueryTimeRange(t time.Time) QueryTimeRange

func NewRangeQueryTimeRange

func NewRangeQueryTimeRange(start time.Time, end time.Time, interval time.Duration) QueryTimeRange

func (*QueryTimeRange) PointIndex

func (q *QueryTimeRange) PointIndex(t int64) int64

PointIndex returns the index in the QueryTimeRange that the timestamp, t, falls on. t must be in line with IntervalMs (ie the step).

type RangeVectorOperator

type RangeVectorOperator interface {
	SeriesOperator

	// StepCount returns the number of time steps produced for each series by this operator.
	// StepCount must only be called after calling SeriesMetadata.
	StepCount() int

	// Range returns the time range selected by this operator at each time step.
	//
	// For example, if this operator represents the selector "some_metric[5m]", Range returns 5 minutes.
	Range() time.Duration

	// NextSeries advances to the next series produced by this operator, or EOS if no more series are available.
	// SeriesMetadata must be called exactly once before calling NextSeries.
	NextSeries(ctx context.Context) error

	// NextStepSamples returns populated RingBuffers with the samples for the next time step for the
	// current series and the timestamps of the next time step, or returns EOS if no more time
	// steps are available.
	NextStepSamples() (*RangeVectorStepData, error)
}

RangeVectorOperator represents all operators that produce range vectors.

type RangeVectorStepData

type RangeVectorStepData struct {
	// Floats contains the float samples for this time step.
	Floats *FPointRingBufferView

	// Histograms contains the histogram samples for this time step.
	//
	// FloatHistogram instances in the buffer must not be modified as they may be returned for subsequent steps.
	// FloatHistogram instances that are retained after the next call to NextStepSamples must be copied, as they
	// may be modified on subsequent calls to NextStepSamples.
	Histograms *HPointRingBufferView

	// StepT is the timestamp of this time step.
	StepT int64

	// RangeStart is the beginning of the time range selected by this time step.
	// RangeStart is inclusive (ie. points with timestamp >= RangeStart are included in the range).
	RangeStart int64

	// RangeEnd is the end of the time range selected by this time step.
	// RangeEnd is the same as StepT except when the @ modifier or offsets are used, in which case
	// RangeEnd reflects the time of the underlying points, and StepT is the timestamp of the point
	// produced by the query.
	// RangeEnd is inclusive (ie. points with timestamp <= RangeEnd are included in the range).
	RangeEnd int64
}

RangeVectorStepData contains the data and timestamps associated with a single time step produced by a RangeVectorOperator.

All timestamps are in milliseconds since the Unix epoch.

For example, if the operator represents the selector "some_metric[5m]", and this time step is for 2024-05-02T00:00:00Z, then:

  • StepT is 1714608000000 (2024-05-02T00:00:00Z)
  • RangeStart is 1714607700000 (2024-05-01T23:55:00Z)
  • RangeEnd is 1714608000000 (2024-05-02T00:00:00Z)

If the operator represents the selector "some_metric[5m] @ 1712016000", and this time step is for 2024-05-02T00:00:00Z, then:

  • StepT is 1714608000000 (2024-05-02T00:00:00Z)
  • RangeStart is 1712015700000 (2024-04-01T23:55:00Z)
  • RangeEnd is 1712016000000 (2024-04-02T00:00:00Z)

type ScalarData

type ScalarData struct {
	// Samples contains floating point samples for this series.
	// Samples must be sorted in timestamp order, earliest timestamps first.
	// Samples must not have duplicate timestamps.
	Samples []promql.FPoint
}

type ScalarOperator

type ScalarOperator interface {
	Operator

	// GetValues returns the samples for this scalar.
	GetValues(ctx context.Context) (ScalarData, error)
}

ScalarOperator represents all operators that produce scalars.

type SeriesMetadata

type SeriesMetadata struct {
	Labels labels.Labels
}

func GetSeriesMetadataSlice

func GetSeriesMetadataSlice(size int) []SeriesMetadata

type SeriesOperator

type SeriesOperator interface {
	Operator

	// SeriesMetadata returns a list of all series that will be returned by this operator.
	// The returned []SeriesMetadata can be modified by the caller or returned to a pool.
	// SeriesMetadata may return series in any order, but the same order must be used by both SeriesMetadata and NextSeries.
	// SeriesMetadata should be called no more than once.
	SeriesMetadata(ctx context.Context) ([]SeriesMetadata, error)
}

SeriesOperator represents all operators that return one or more series.

type StringOperator

type StringOperator interface {
	Operator

	// GetValue returns the string
	GetValue() string
}

StringOperator represents all operators that produce strings.

Jump to

Keyboard shortcuts

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