Add warnings (and annotations) to PromQL query results (#12152)

Return annotations (warnings and infos) from PromQL queries

This generalizes the warnings we have already used before (but only for problems with remote read) as "annotations".

Annotations can be warnings or infos (the latter could be false positives). We do not treat them different in the API for now and return them all as "warnings". It would be easy to distinguish them and return infos separately, should that appear useful in the future.

The new annotations are then used to create a lot of warnings or infos during PromQL evaluations. Partially these are things we have wanted for a long time (e.g. inform the user that they have applied `rate` to a metric that doesn't look like a counter), but the new native histograms have created even more needs for those annotations (e.g. if a query tries to aggregate float numbers with histograms).

The annotations added here are not yet complete. A prominent example would be a warning about a range too short for a rate calculation. But such a warnings is more tricky to create with good fidelity and we will tackle it later.

Another TODO is to take annotations into account when evaluating recording rules.

---------

Signed-off-by: Jeanette Tan <jeanette.tan@grafana.com>
This commit is contained in:
zenador 2023-09-15 00:57:31 +08:00 committed by GitHub
parent 156222cc50
commit 69edd8709b
No known key found for this signature in database
GPG key ID: 4AEE18F83AFDEB23
34 changed files with 1204 additions and 959 deletions

View file

@ -662,7 +662,7 @@ func dumpSamples(ctx context.Context, path string, mint, maxt int64, match strin
} }
if ws := ss.Warnings(); len(ws) > 0 { if ws := ss.Warnings(); len(ws) > 0 {
return tsdb_errors.NewMulti(ws...).Err() return tsdb_errors.NewMulti(ws.AsErrors()...).Err()
} }
if ss.Err() != nil { if ss.Err() != nil {

View file

@ -44,6 +44,7 @@ import (
"github.com/prometheus/prometheus/promql/parser" "github.com/prometheus/prometheus/promql/parser"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/util/annotations"
"github.com/prometheus/prometheus/util/stats" "github.com/prometheus/prometheus/util/stats"
"github.com/prometheus/prometheus/util/zeropool" "github.com/prometheus/prometheus/util/zeropool"
) )
@ -573,7 +574,7 @@ func (ng *Engine) newTestQuery(f func(context.Context) error) Query {
// //
// At this point per query only one EvalStmt is evaluated. Alert and record // At this point per query only one EvalStmt is evaluated. Alert and record
// statements are not handled by the Engine. // statements are not handled by the Engine.
func (ng *Engine) exec(ctx context.Context, q *query) (v parser.Value, ws storage.Warnings, err error) { func (ng *Engine) exec(ctx context.Context, q *query) (v parser.Value, ws annotations.Annotations, err error) {
ng.metrics.currentQueries.Inc() ng.metrics.currentQueries.Inc()
defer func() { defer func() {
ng.metrics.currentQueries.Dec() ng.metrics.currentQueries.Dec()
@ -666,7 +667,7 @@ func durationMilliseconds(d time.Duration) int64 {
} }
// execEvalStmt evaluates the expression of an evaluation statement for the given time range. // execEvalStmt evaluates the expression of an evaluation statement for the given time range.
func (ng *Engine) execEvalStmt(ctx context.Context, query *query, s *parser.EvalStmt) (parser.Value, storage.Warnings, error) { func (ng *Engine) execEvalStmt(ctx context.Context, query *query, s *parser.EvalStmt) (parser.Value, annotations.Annotations, error) {
prepareSpanTimer, ctxPrepare := query.stats.GetSpanTimer(ctx, stats.QueryPreparationTime, ng.metrics.queryPrepareTime) prepareSpanTimer, ctxPrepare := query.stats.GetSpanTimer(ctx, stats.QueryPreparationTime, ng.metrics.queryPrepareTime)
mint, maxt := ng.findMinMaxTime(s) mint, maxt := ng.findMinMaxTime(s)
querier, err := query.queryable.Querier(mint, maxt) querier, err := query.queryable.Querier(mint, maxt)
@ -952,7 +953,7 @@ func extractGroupsFromPath(p []parser.Node) (bool, []string) {
return false, nil return false, nil
} }
func checkAndExpandSeriesSet(ctx context.Context, expr parser.Expr) (storage.Warnings, error) { func checkAndExpandSeriesSet(ctx context.Context, expr parser.Expr) (annotations.Annotations, error) {
switch e := expr.(type) { switch e := expr.(type) {
case *parser.MatrixSelector: case *parser.MatrixSelector:
return checkAndExpandSeriesSet(ctx, e.VectorSelector) return checkAndExpandSeriesSet(ctx, e.VectorSelector)
@ -967,7 +968,7 @@ func checkAndExpandSeriesSet(ctx context.Context, expr parser.Expr) (storage.War
return nil, nil return nil, nil
} }
func expandSeriesSet(ctx context.Context, it storage.SeriesSet) (res []storage.Series, ws storage.Warnings, err error) { func expandSeriesSet(ctx context.Context, it storage.SeriesSet) (res []storage.Series, ws annotations.Annotations, err error) {
for it.Next() { for it.Next() {
select { select {
case <-ctx.Done(): case <-ctx.Done():
@ -981,7 +982,7 @@ func expandSeriesSet(ctx context.Context, it storage.SeriesSet) (res []storage.S
type errWithWarnings struct { type errWithWarnings struct {
err error err error
warnings storage.Warnings warnings annotations.Annotations
} }
func (e errWithWarnings) Error() string { return e.err.Error() } func (e errWithWarnings) Error() string { return e.err.Error() }
@ -1016,7 +1017,7 @@ func (ev *evaluator) error(err error) {
} }
// recover is the handler that turns panics into returns from the top level of evaluation. // recover is the handler that turns panics into returns from the top level of evaluation.
func (ev *evaluator) recover(expr parser.Expr, ws *storage.Warnings, errp *error) { func (ev *evaluator) recover(expr parser.Expr, ws *annotations.Annotations, errp *error) {
e := recover() e := recover()
if e == nil { if e == nil {
return return
@ -1032,7 +1033,7 @@ func (ev *evaluator) recover(expr parser.Expr, ws *storage.Warnings, errp *error
*errp = fmt.Errorf("unexpected error: %w", err) *errp = fmt.Errorf("unexpected error: %w", err)
case errWithWarnings: case errWithWarnings:
*errp = err.err *errp = err.err
*ws = append(*ws, err.warnings...) ws.Merge(err.warnings)
case error: case error:
*errp = err *errp = err
default: default:
@ -1040,7 +1041,7 @@ func (ev *evaluator) recover(expr parser.Expr, ws *storage.Warnings, errp *error
} }
} }
func (ev *evaluator) Eval(expr parser.Expr) (v parser.Value, ws storage.Warnings, err error) { func (ev *evaluator) Eval(expr parser.Expr) (v parser.Value, ws annotations.Annotations, err error) {
defer ev.recover(expr, &ws, &err) defer ev.recover(expr, &ws, &err)
v, ws = ev.eval(expr) v, ws = ev.eval(expr)
@ -1109,19 +1110,19 @@ func (enh *EvalNodeHelper) DropMetricName(l labels.Labels) labels.Labels {
// function call results. // function call results.
// The prepSeries function (if provided) can be used to prepare the helper // The prepSeries function (if provided) can be used to prepare the helper
// for each series, then passed to each call funcCall. // for each series, then passed to each call funcCall.
func (ev *evaluator) rangeEval(prepSeries func(labels.Labels, *EvalSeriesHelper), funcCall func([]parser.Value, [][]EvalSeriesHelper, *EvalNodeHelper) (Vector, storage.Warnings), exprs ...parser.Expr) (Matrix, storage.Warnings) { func (ev *evaluator) rangeEval(prepSeries func(labels.Labels, *EvalSeriesHelper), funcCall func([]parser.Value, [][]EvalSeriesHelper, *EvalNodeHelper) (Vector, annotations.Annotations), exprs ...parser.Expr) (Matrix, annotations.Annotations) {
numSteps := int((ev.endTimestamp-ev.startTimestamp)/ev.interval) + 1 numSteps := int((ev.endTimestamp-ev.startTimestamp)/ev.interval) + 1
matrixes := make([]Matrix, len(exprs)) matrixes := make([]Matrix, len(exprs))
origMatrixes := make([]Matrix, len(exprs)) origMatrixes := make([]Matrix, len(exprs))
originalNumSamples := ev.currentSamples originalNumSamples := ev.currentSamples
var warnings storage.Warnings var warnings annotations.Annotations
for i, e := range exprs { for i, e := range exprs {
// Functions will take string arguments from the expressions, not the values. // Functions will take string arguments from the expressions, not the values.
if e != nil && e.Type() != parser.ValueTypeString { if e != nil && e.Type() != parser.ValueTypeString {
// ev.currentSamples will be updated to the correct value within the ev.eval call. // ev.currentSamples will be updated to the correct value within the ev.eval call.
val, ws := ev.eval(e) val, ws := ev.eval(e)
warnings = append(warnings, ws...) warnings.Merge(ws)
matrixes[i] = val.(Matrix) matrixes[i] = val.(Matrix)
// Keep a copy of the original point slices so that they // Keep a copy of the original point slices so that they
@ -1233,7 +1234,7 @@ func (ev *evaluator) rangeEval(prepSeries func(labels.Labels, *EvalSeriesHelper)
enh.Ts = ts enh.Ts = ts
result, ws := funcCall(args, bufHelpers, enh) result, ws := funcCall(args, bufHelpers, enh)
enh.Out = result[:0] // Reuse result vector. enh.Out = result[:0] // Reuse result vector.
warnings = append(warnings, ws...) warnings.Merge(ws)
ev.currentSamples += len(result) ev.currentSamples += len(result)
// When we reset currentSamples to tempNumSamples during the next iteration of the loop it also // When we reset currentSamples to tempNumSamples during the next iteration of the loop it also
@ -1310,7 +1311,7 @@ func (ev *evaluator) rangeEval(prepSeries func(labels.Labels, *EvalSeriesHelper)
// evalSubquery evaluates given SubqueryExpr and returns an equivalent // evalSubquery evaluates given SubqueryExpr and returns an equivalent
// evaluated MatrixSelector in its place. Note that the Name and LabelMatchers are not set. // evaluated MatrixSelector in its place. Note that the Name and LabelMatchers are not set.
func (ev *evaluator) evalSubquery(subq *parser.SubqueryExpr) (*parser.MatrixSelector, int, storage.Warnings) { func (ev *evaluator) evalSubquery(subq *parser.SubqueryExpr) (*parser.MatrixSelector, int, annotations.Annotations) {
samplesStats := ev.samplesStats samplesStats := ev.samplesStats
// Avoid double counting samples when running a subquery, those samples will be counted in later stage. // Avoid double counting samples when running a subquery, those samples will be counted in later stage.
ev.samplesStats = ev.samplesStats.NewChild() ev.samplesStats = ev.samplesStats.NewChild()
@ -1343,7 +1344,7 @@ func (ev *evaluator) evalSubquery(subq *parser.SubqueryExpr) (*parser.MatrixSele
} }
// eval evaluates the given expression as the given AST expression node requires. // eval evaluates the given expression as the given AST expression node requires.
func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) { func (ev *evaluator) eval(expr parser.Expr) (parser.Value, annotations.Annotations) {
// This is the top-level evaluation method. // This is the top-level evaluation method.
// Thus, we check for timeout/cancellation here. // Thus, we check for timeout/cancellation here.
if err := contextDone(ev.ctx, "expression evaluation"); err != nil { if err := contextDone(ev.ctx, "expression evaluation"); err != nil {
@ -1372,17 +1373,17 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
param := unwrapStepInvariantExpr(e.Param) param := unwrapStepInvariantExpr(e.Param)
unwrapParenExpr(&param) unwrapParenExpr(&param)
if s, ok := param.(*parser.StringLiteral); ok { if s, ok := param.(*parser.StringLiteral); ok {
return ev.rangeEval(initSeries, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(initSeries, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.aggregation(e.Op, sortedGrouping, e.Without, s.Val, v[0].(Vector), sh[0], enh), nil return ev.aggregation(e, sortedGrouping, s.Val, v[0].(Vector), sh[0], enh)
}, e.Expr) }, e.Expr)
} }
return ev.rangeEval(initSeries, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(initSeries, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
var param float64 var param float64
if e.Param != nil { if e.Param != nil {
param = v[0].(Vector)[0].F param = v[0].(Vector)[0].F
} }
return ev.aggregation(e.Op, sortedGrouping, e.Without, param, v[1].(Vector), sh[1], enh), nil return ev.aggregation(e, sortedGrouping, param, v[1].(Vector), sh[1], enh)
}, e.Param, e.Expr) }, e.Param, e.Expr)
case *parser.Call: case *parser.Call:
@ -1404,7 +1405,7 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
var ( var (
matrixArgIndex int matrixArgIndex int
matrixArg bool matrixArg bool
warnings storage.Warnings warnings annotations.Annotations
) )
for i := range e.Args { for i := range e.Args {
unwrapParenExpr(&e.Args[i]) unwrapParenExpr(&e.Args[i])
@ -1422,7 +1423,7 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
// Replacing parser.SubqueryExpr with parser.MatrixSelector. // Replacing parser.SubqueryExpr with parser.MatrixSelector.
val, totalSamples, ws := ev.evalSubquery(subq) val, totalSamples, ws := ev.evalSubquery(subq)
e.Args[i] = val e.Args[i] = val
warnings = append(warnings, ws...) warnings.Merge(ws)
defer func() { defer func() {
// subquery result takes space in the memory. Get rid of that at the end. // subquery result takes space in the memory. Get rid of that at the end.
val.VectorSelector.(*parser.VectorSelector).Series = nil val.VectorSelector.(*parser.VectorSelector).Series = nil
@ -1433,8 +1434,9 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
} }
if !matrixArg { if !matrixArg {
// Does not have a matrix argument. // Does not have a matrix argument.
return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return call(v, e.Args, enh), warnings vec, annos := call(v, e.Args, enh)
return vec, warnings.Merge(annos)
}, e.Args...) }, e.Args...)
} }
@ -1448,7 +1450,7 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
otherArgs[i] = val.(Matrix) otherArgs[i] = val.(Matrix)
otherInArgs[i] = Vector{Sample{}} otherInArgs[i] = Vector{Sample{}}
inArgs[i] = otherInArgs[i] inArgs[i] = otherInArgs[i]
warnings = append(warnings, ws...) warnings.Merge(ws)
} }
} }
@ -1459,7 +1461,7 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
selVS := sel.VectorSelector.(*parser.VectorSelector) selVS := sel.VectorSelector.(*parser.VectorSelector)
ws, err := checkAndExpandSeriesSet(ev.ctx, sel) ws, err := checkAndExpandSeriesSet(ev.ctx, sel)
warnings = append(warnings, ws...) warnings.Merge(ws)
if err != nil { if err != nil {
ev.error(errWithWarnings{fmt.Errorf("expanding series: %w", err), warnings}) ev.error(errWithWarnings{fmt.Errorf("expanding series: %w", err), warnings})
} }
@ -1522,8 +1524,10 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
inMatrix[0].Histograms = histograms inMatrix[0].Histograms = histograms
enh.Ts = ts enh.Ts = ts
// Make the function call. // Make the function call.
outVec := call(inArgs, e.Args, enh) outVec, annos := call(inArgs, e.Args, enh)
warnings.Merge(annos)
ev.samplesStats.IncrementSamplesAtStep(step, int64(len(floats)+len(histograms))) ev.samplesStats.IncrementSamplesAtStep(step, int64(len(floats)+len(histograms)))
enh.Out = outVec[:0] enh.Out = outVec[:0]
if len(outVec) > 0 { if len(outVec) > 0 {
if outVec[0].H == nil { if outVec[0].H == nil {
@ -1626,7 +1630,7 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
case *parser.BinaryExpr: case *parser.BinaryExpr:
switch lt, rt := e.LHS.Type(), e.RHS.Type(); { switch lt, rt := e.LHS.Type(), e.RHS.Type(); {
case lt == parser.ValueTypeScalar && rt == parser.ValueTypeScalar: case lt == parser.ValueTypeScalar && rt == parser.ValueTypeScalar:
return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
val := scalarBinop(e.Op, v[0].(Vector)[0].F, v[1].(Vector)[0].F) val := scalarBinop(e.Op, v[0].(Vector)[0].F, v[1].(Vector)[0].F)
return append(enh.Out, Sample{F: val}), nil return append(enh.Out, Sample{F: val}), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
@ -1639,36 +1643,36 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
} }
switch e.Op { switch e.Op {
case parser.LAND: case parser.LAND:
return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.VectorAnd(v[0].(Vector), v[1].(Vector), e.VectorMatching, sh[0], sh[1], enh), nil return ev.VectorAnd(v[0].(Vector), v[1].(Vector), e.VectorMatching, sh[0], sh[1], enh), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
case parser.LOR: case parser.LOR:
return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.VectorOr(v[0].(Vector), v[1].(Vector), e.VectorMatching, sh[0], sh[1], enh), nil return ev.VectorOr(v[0].(Vector), v[1].(Vector), e.VectorMatching, sh[0], sh[1], enh), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
case parser.LUNLESS: case parser.LUNLESS:
return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.VectorUnless(v[0].(Vector), v[1].(Vector), e.VectorMatching, sh[0], sh[1], enh), nil return ev.VectorUnless(v[0].(Vector), v[1].(Vector), e.VectorMatching, sh[0], sh[1], enh), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
default: default:
return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(initSignatures, func(v []parser.Value, sh [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.VectorBinop(e.Op, v[0].(Vector), v[1].(Vector), e.VectorMatching, e.ReturnBool, sh[0], sh[1], enh), nil return ev.VectorBinop(e.Op, v[0].(Vector), v[1].(Vector), e.VectorMatching, e.ReturnBool, sh[0], sh[1], enh), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
} }
case lt == parser.ValueTypeVector && rt == parser.ValueTypeScalar: case lt == parser.ValueTypeVector && rt == parser.ValueTypeScalar:
return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.VectorscalarBinop(e.Op, v[0].(Vector), Scalar{V: v[1].(Vector)[0].F}, false, e.ReturnBool, enh), nil return ev.VectorscalarBinop(e.Op, v[0].(Vector), Scalar{V: v[1].(Vector)[0].F}, false, e.ReturnBool, enh), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
case lt == parser.ValueTypeScalar && rt == parser.ValueTypeVector: case lt == parser.ValueTypeScalar && rt == parser.ValueTypeVector:
return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return ev.VectorscalarBinop(e.Op, v[1].(Vector), Scalar{V: v[0].(Vector)[0].F}, true, e.ReturnBool, enh), nil return ev.VectorscalarBinop(e.Op, v[1].(Vector), Scalar{V: v[0].(Vector)[0].F}, true, e.ReturnBool, enh), nil
}, e.LHS, e.RHS) }, e.LHS, e.RHS)
} }
case *parser.NumberLiteral: case *parser.NumberLiteral:
return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
return append(enh.Out, Sample{F: e.Val, Metric: labels.EmptyLabels()}), nil return append(enh.Out, Sample{F: e.Val, Metric: labels.EmptyLabels()}), nil
}) })
@ -1834,7 +1838,7 @@ func (ev *evaluator) eval(expr parser.Expr) (parser.Value, storage.Warnings) {
panic(fmt.Errorf("unhandled expression of type: %T", expr)) panic(fmt.Errorf("unhandled expression of type: %T", expr))
} }
func (ev *evaluator) rangeEvalTimestampFunctionOverVectorSelector(vs *parser.VectorSelector, call FunctionCall, e *parser.Call) (parser.Value, storage.Warnings) { func (ev *evaluator) rangeEvalTimestampFunctionOverVectorSelector(vs *parser.VectorSelector, call FunctionCall, e *parser.Call) (parser.Value, annotations.Annotations) {
ws, err := checkAndExpandSeriesSet(ev.ctx, vs) ws, err := checkAndExpandSeriesSet(ev.ctx, vs)
if err != nil { if err != nil {
ev.error(errWithWarnings{fmt.Errorf("expanding series: %w", err), ws}) ev.error(errWithWarnings{fmt.Errorf("expanding series: %w", err), ws})
@ -1846,7 +1850,7 @@ func (ev *evaluator) rangeEvalTimestampFunctionOverVectorSelector(vs *parser.Vec
seriesIterators[i] = storage.NewMemoizedIterator(it, durationMilliseconds(ev.lookbackDelta)) seriesIterators[i] = storage.NewMemoizedIterator(it, durationMilliseconds(ev.lookbackDelta))
} }
return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, storage.Warnings) { return ev.rangeEval(nil, func(v []parser.Value, _ [][]EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
if vs.Timestamp != nil { if vs.Timestamp != nil {
// This is a special case for "timestamp()" when the @ modifier is used, to ensure that // This is a special case for "timestamp()" when the @ modifier is used, to ensure that
// we return a point for each time step in this case. // we return a point for each time step in this case.
@ -1874,7 +1878,8 @@ func (ev *evaluator) rangeEvalTimestampFunctionOverVectorSelector(vs *parser.Vec
} }
} }
ev.samplesStats.UpdatePeak(ev.currentSamples) ev.samplesStats.UpdatePeak(ev.currentSamples)
return call([]parser.Value{vec}, e.Args, enh), ws vec, annos := call([]parser.Value{vec}, e.Args, enh)
return vec, ws.Merge(annos)
}) })
} }
@ -1945,7 +1950,7 @@ func putHPointSlice(p []HPoint) {
} }
// matrixSelector evaluates a *parser.MatrixSelector expression. // matrixSelector evaluates a *parser.MatrixSelector expression.
func (ev *evaluator) matrixSelector(node *parser.MatrixSelector) (Matrix, storage.Warnings) { func (ev *evaluator) matrixSelector(node *parser.MatrixSelector) (Matrix, annotations.Annotations) {
var ( var (
vs = node.VectorSelector.(*parser.VectorSelector) vs = node.VectorSelector.(*parser.VectorSelector)
@ -2525,7 +2530,10 @@ type groupedAggregation struct {
// aggregation evaluates an aggregation operation on a Vector. The provided grouping labels // aggregation evaluates an aggregation operation on a Vector. The provided grouping labels
// must be sorted. // must be sorted.
func (ev *evaluator) aggregation(op parser.ItemType, grouping []string, without bool, param interface{}, vec Vector, seriesHelper []EvalSeriesHelper, enh *EvalNodeHelper) Vector { func (ev *evaluator) aggregation(e *parser.AggregateExpr, grouping []string, param interface{}, vec Vector, seriesHelper []EvalSeriesHelper, enh *EvalNodeHelper) (Vector, annotations.Annotations) {
op := e.Op
without := e.Without
annos := annotations.Annotations{}
result := map[uint64]*groupedAggregation{} result := map[uint64]*groupedAggregation{}
orderedResult := []*groupedAggregation{} orderedResult := []*groupedAggregation{}
var k int64 var k int64
@ -2536,7 +2544,7 @@ func (ev *evaluator) aggregation(op parser.ItemType, grouping []string, without
} }
k = int64(f) k = int64(f)
if k < 1 { if k < 1 {
return Vector{} return Vector{}, annos
} }
} }
var q float64 var q float64
@ -2789,7 +2797,8 @@ func (ev *evaluator) aggregation(op parser.ItemType, grouping []string, without
case parser.AVG: case parser.AVG:
if aggr.hasFloat && aggr.hasHistogram { if aggr.hasFloat && aggr.hasHistogram {
// We cannot aggregate histogram sample with a float64 sample. // We cannot aggregate histogram sample with a float64 sample.
// TODO(zenador): Issue warning when plumbing is in place. metricName := aggr.labels.Get(labels.MetricName)
annos.Add(annotations.NewMixedFloatsHistogramsWarning(metricName, e.Expr.PositionRange()))
continue continue
} }
if aggr.hasHistogram { if aggr.hasHistogram {
@ -2834,12 +2843,16 @@ func (ev *evaluator) aggregation(op parser.ItemType, grouping []string, without
continue // Bypass default append. continue // Bypass default append.
case parser.QUANTILE: case parser.QUANTILE:
if math.IsNaN(q) || q < 0 || q > 1 {
annos.Add(annotations.NewInvalidQuantileWarning(q, e.Param.PositionRange()))
}
aggr.floatValue = quantile(q, aggr.heap) aggr.floatValue = quantile(q, aggr.heap)
case parser.SUM: case parser.SUM:
if aggr.hasFloat && aggr.hasHistogram { if aggr.hasFloat && aggr.hasHistogram {
// We cannot aggregate histogram sample with a float64 sample. // We cannot aggregate histogram sample with a float64 sample.
// TODO(zenador): Issue warning when plumbing is in place. metricName := aggr.labels.Get(labels.MetricName)
annos.Add(annotations.NewMixedFloatsHistogramsWarning(metricName, e.Expr.PositionRange()))
continue continue
} }
if aggr.hasHistogram { if aggr.hasHistogram {
@ -2855,7 +2868,7 @@ func (ev *evaluator) aggregation(op parser.ItemType, grouping []string, without
H: aggr.histogramValue, H: aggr.histogramValue,
}) })
} }
return enh.Out return enh.Out, annos
} }
// groupingKey builds and returns the grouping key for the given metric and // groupingKey builds and returns the grouping key for the given metric and

View file

@ -32,8 +32,10 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/model/timestamp" "github.com/prometheus/prometheus/model/timestamp"
"github.com/prometheus/prometheus/promql/parser" "github.com/prometheus/prometheus/promql/parser"
"github.com/prometheus/prometheus/promql/parser/posrange"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/tsdb/tsdbutil" "github.com/prometheus/prometheus/tsdb/tsdbutil"
"github.com/prometheus/prometheus/util/annotations"
"github.com/prometheus/prometheus/util/stats" "github.com/prometheus/prometheus/util/stats"
"github.com/prometheus/prometheus/util/teststorage" "github.com/prometheus/prometheus/util/teststorage"
) )
@ -198,11 +200,11 @@ func (q *errQuerier) Select(context.Context, bool, *storage.SelectHints, ...*lab
return errSeriesSet{err: q.err} return errSeriesSet{err: q.err}
} }
func (*errQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (*errQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
func (*errQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (*errQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
func (*errQuerier) Close() error { return nil } func (*errQuerier) Close() error { return nil }
@ -212,10 +214,10 @@ type errSeriesSet struct {
err error err error
} }
func (errSeriesSet) Next() bool { return false } func (errSeriesSet) Next() bool { return false }
func (errSeriesSet) At() storage.Series { return nil } func (errSeriesSet) At() storage.Series { return nil }
func (e errSeriesSet) Err() error { return e.err } func (e errSeriesSet) Err() error { return e.err }
func (e errSeriesSet) Warnings() storage.Warnings { return nil } func (e errSeriesSet) Warnings() annotations.Annotations { return nil }
func TestQueryError(t *testing.T) { func TestQueryError(t *testing.T) {
opts := EngineOpts{ opts := EngineOpts{
@ -1675,9 +1677,9 @@ func TestRecoverEvaluatorError(t *testing.T) {
func TestRecoverEvaluatorErrorWithWarnings(t *testing.T) { func TestRecoverEvaluatorErrorWithWarnings(t *testing.T) {
ev := &evaluator{logger: log.NewNopLogger()} ev := &evaluator{logger: log.NewNopLogger()}
var err error var err error
var ws storage.Warnings var ws annotations.Annotations
warnings := storage.Warnings{errors.New("custom warning")} warnings := annotations.New().Add(errors.New("custom warning"))
e := errWithWarnings{ e := errWithWarnings{
err: errors.New("custom error"), err: errors.New("custom error"),
warnings: warnings, warnings: warnings,
@ -2146,7 +2148,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
expected: &parser.StepInvariantExpr{ expected: &parser.StepInvariantExpr{
Expr: &parser.NumberLiteral{ Expr: &parser.NumberLiteral{
Val: 123.4567, Val: 123.4567,
PosRange: parser.PositionRange{Start: 0, End: 8}, PosRange: posrange.PositionRange{Start: 0, End: 8},
}, },
}, },
}, },
@ -2155,7 +2157,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
expected: &parser.StepInvariantExpr{ expected: &parser.StepInvariantExpr{
Expr: &parser.StringLiteral{ Expr: &parser.StringLiteral{
Val: "foo", Val: "foo",
PosRange: parser.PositionRange{Start: 0, End: 5}, PosRange: posrange.PositionRange{Start: 0, End: 5},
}, },
}, },
}, },
@ -2168,7 +2170,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 3, End: 3,
}, },
@ -2178,7 +2180,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 6, Start: 6,
End: 9, End: 9,
}, },
@ -2195,7 +2197,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 3, End: 3,
}, },
@ -2206,7 +2208,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 6, Start: 6,
End: 14, End: 14,
}, },
@ -2226,7 +2228,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 8, End: 8,
}, },
@ -2237,7 +2239,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 11, Start: 11,
End: 19, End: 19,
}, },
@ -2255,7 +2257,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 4, End: 4,
}, },
@ -2275,7 +2277,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
parser.MustLabelMatcher(labels.MatchEqual, "a", "b"), parser.MustLabelMatcher(labels.MatchEqual, "a", "b"),
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 11, End: 11,
}, },
@ -2294,13 +2296,13 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 13, Start: 13,
End: 24, End: 24,
}, },
}, },
Grouping: []string{"foo"}, Grouping: []string{"foo"},
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 25, End: 25,
}, },
@ -2316,14 +2318,14 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 13, Start: 13,
End: 29, End: 29,
}, },
Timestamp: makeInt64Pointer(10000), Timestamp: makeInt64Pointer(10000),
}, },
Grouping: []string{"foo"}, Grouping: []string{"foo"},
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 30, End: 30,
}, },
@ -2343,13 +2345,13 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric1"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric1"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 4, Start: 4,
End: 21, End: 21,
}, },
Timestamp: makeInt64Pointer(10000), Timestamp: makeInt64Pointer(10000),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 22, End: 22,
}, },
@ -2361,13 +2363,13 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric2"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric2"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 29, Start: 29,
End: 46, End: 46,
}, },
Timestamp: makeInt64Pointer(20000), Timestamp: makeInt64Pointer(20000),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 25, Start: 25,
End: 47, End: 47,
}, },
@ -2387,7 +2389,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 11, End: 11,
}, },
@ -2404,7 +2406,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 29, Start: 29,
End: 40, End: 40,
}, },
@ -2414,19 +2416,19 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
EndPos: 49, EndPos: 49,
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 24, Start: 24,
End: 50, End: 50,
}, },
}, },
Param: &parser.NumberLiteral{ Param: &parser.NumberLiteral{
Val: 5, Val: 5,
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 21, Start: 21,
End: 22, End: 22,
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 16, Start: 16,
End: 51, End: 51,
}, },
@ -2439,7 +2441,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
expected: &parser.Call{ expected: &parser.Call{
Func: parser.MustGetFunction("time"), Func: parser.MustGetFunction("time"),
Args: parser.Expressions{}, Args: parser.Expressions{},
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 6, End: 6,
}, },
@ -2454,7 +2456,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"), parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"),
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 14, End: 14,
}, },
@ -2474,7 +2476,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"), parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"),
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 14, End: 14,
}, },
@ -2499,13 +2501,13 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"), parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"),
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 4, Start: 4,
End: 23, End: 23,
}, },
Timestamp: makeInt64Pointer(20000), Timestamp: makeInt64Pointer(20000),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 24, End: 24,
}, },
@ -2536,7 +2538,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"), parser.MustLabelMatcher(labels.MatchEqual, "bar", "baz"),
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 19, Start: 19,
End: 33, End: 33,
}, },
@ -2545,7 +2547,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
EndPos: 37, EndPos: 37,
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 14, Start: 14,
End: 38, End: 38,
}, },
@ -2555,7 +2557,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
EndPos: 56, EndPos: 56,
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 57, End: 57,
}, },
@ -2575,7 +2577,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 27, End: 27,
}, },
@ -2597,7 +2599,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 11, End: 11,
}, },
@ -2625,7 +2627,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 1, Start: 1,
End: 4, End: 4,
}, },
@ -2638,14 +2640,14 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "bar"),
}, },
Timestamp: makeInt64Pointer(1234000), Timestamp: makeInt64Pointer(1234000),
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 7, Start: 7,
End: 27, End: 27,
}, },
}, },
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 28, End: 28,
}, },
@ -2676,18 +2678,18 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 8, Start: 8,
End: 19, End: 19,
}, },
Timestamp: makeInt64Pointer(10000), Timestamp: makeInt64Pointer(10000),
}}, }},
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 4, Start: 4,
End: 20, End: 20,
}, },
}}, }},
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 21, End: 21,
}, },
@ -2709,13 +2711,13 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric1"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric1"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 8, Start: 8,
End: 25, End: 25,
}, },
Timestamp: makeInt64Pointer(10000), Timestamp: makeInt64Pointer(10000),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 4, Start: 4,
End: 26, End: 26,
}, },
@ -2727,19 +2729,19 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric2"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric2"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 33, Start: 33,
End: 50, End: 50,
}, },
Timestamp: makeInt64Pointer(20000), Timestamp: makeInt64Pointer(20000),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 29, Start: 29,
End: 52, End: 52,
}, },
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 52, End: 52,
}, },
@ -2754,7 +2756,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 13, End: 13,
}, },
@ -2771,7 +2773,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "foo"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 11, End: 11,
}, },
@ -2791,7 +2793,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 4, End: 4,
}, },
@ -2812,7 +2814,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "test"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 4, End: 4,
}, },
@ -2831,7 +2833,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 11, End: 11,
}, },
@ -2853,7 +2855,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 11, End: 11,
}, },
@ -2883,7 +2885,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
LabelMatchers: []*labels.Matcher{ LabelMatchers: []*labels.Matcher{
parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"), parser.MustLabelMatcher(labels.MatchEqual, "__name__", "some_metric"),
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 6, Start: 6,
End: 17, End: 17,
}, },
@ -2894,20 +2896,20 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
Op: parser.MUL, Op: parser.MUL,
LHS: &parser.NumberLiteral{ LHS: &parser.NumberLiteral{
Val: 3, Val: 3,
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 21, Start: 21,
End: 22, End: 22,
}, },
}, },
RHS: &parser.NumberLiteral{ RHS: &parser.NumberLiteral{
Val: 1024, Val: 1024,
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 25, Start: 25,
End: 29, End: 29,
}, },
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 20, Start: 20,
End: 30, End: 30,
}, },
@ -2915,7 +2917,7 @@ func TestPreprocessAndWrapWithStepInvariantExpr(t *testing.T) {
}, },
}, },
}, },
PosRange: parser.PositionRange{ PosRange: posrange.PositionRange{
Start: 0, Start: 0,
End: 31, End: 31,
}, },

File diff suppressed because it is too large Load diff

View file

@ -20,6 +20,8 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/promql/parser/posrange"
) )
// Node is a generic interface for all nodes in an AST. // Node is a generic interface for all nodes in an AST.
@ -45,7 +47,7 @@ type Node interface {
Pretty(level int) string Pretty(level int) string
// PositionRange returns the position of the AST Node in the query string. // PositionRange returns the position of the AST Node in the query string.
PositionRange() PositionRange PositionRange() posrange.PositionRange
} }
// Statement is a generic interface for all statements. // Statement is a generic interface for all statements.
@ -94,7 +96,7 @@ type AggregateExpr struct {
Param Expr // Parameter used by some aggregators. Param Expr // Parameter used by some aggregators.
Grouping []string // The labels by which to group the Vector. Grouping []string // The labels by which to group the Vector.
Without bool // Whether to drop the given labels rather than keep them. Without bool // Whether to drop the given labels rather than keep them.
PosRange PositionRange PosRange posrange.PositionRange
} }
// BinaryExpr represents a binary expression between two child expressions. // BinaryExpr represents a binary expression between two child expressions.
@ -115,7 +117,7 @@ type Call struct {
Func *Function // The function that was called. Func *Function // The function that was called.
Args Expressions // Arguments used in the call. Args Expressions // Arguments used in the call.
PosRange PositionRange PosRange posrange.PositionRange
} }
// MatrixSelector represents a Matrix selection. // MatrixSelector represents a Matrix selection.
@ -125,7 +127,7 @@ type MatrixSelector struct {
VectorSelector Expr VectorSelector Expr
Range time.Duration Range time.Duration
EndPos Pos EndPos posrange.Pos
} }
// SubqueryExpr represents a subquery. // SubqueryExpr represents a subquery.
@ -143,27 +145,27 @@ type SubqueryExpr struct {
StartOrEnd ItemType // Set when @ is used with start() or end() StartOrEnd ItemType // Set when @ is used with start() or end()
Step time.Duration Step time.Duration
EndPos Pos EndPos posrange.Pos
} }
// NumberLiteral represents a number. // NumberLiteral represents a number.
type NumberLiteral struct { type NumberLiteral struct {
Val float64 Val float64
PosRange PositionRange PosRange posrange.PositionRange
} }
// ParenExpr wraps an expression so it cannot be disassembled as a consequence // ParenExpr wraps an expression so it cannot be disassembled as a consequence
// of operator precedence. // of operator precedence.
type ParenExpr struct { type ParenExpr struct {
Expr Expr Expr Expr
PosRange PositionRange PosRange posrange.PositionRange
} }
// StringLiteral represents a string. // StringLiteral represents a string.
type StringLiteral struct { type StringLiteral struct {
Val string Val string
PosRange PositionRange PosRange posrange.PositionRange
} }
// UnaryExpr represents a unary operation on another expression. // UnaryExpr represents a unary operation on another expression.
@ -172,7 +174,7 @@ type UnaryExpr struct {
Op ItemType Op ItemType
Expr Expr Expr Expr
StartPos Pos StartPos posrange.Pos
} }
// StepInvariantExpr represents a query which evaluates to the same result // StepInvariantExpr represents a query which evaluates to the same result
@ -184,7 +186,9 @@ type StepInvariantExpr struct {
func (e *StepInvariantExpr) String() string { return e.Expr.String() } func (e *StepInvariantExpr) String() string { return e.Expr.String() }
func (e *StepInvariantExpr) PositionRange() PositionRange { return e.Expr.PositionRange() } func (e *StepInvariantExpr) PositionRange() posrange.PositionRange {
return e.Expr.PositionRange()
}
// VectorSelector represents a Vector selection. // VectorSelector represents a Vector selection.
type VectorSelector struct { type VectorSelector struct {
@ -204,7 +208,7 @@ type VectorSelector struct {
UnexpandedSeriesSet storage.SeriesSet UnexpandedSeriesSet storage.SeriesSet
Series []storage.Series Series []storage.Series
PosRange PositionRange PosRange posrange.PositionRange
} }
// TestStmt is an internal helper statement that allows execution // TestStmt is an internal helper statement that allows execution
@ -215,8 +219,8 @@ func (TestStmt) String() string { return "test statement" }
func (TestStmt) PromQLStmt() {} func (TestStmt) PromQLStmt() {}
func (t TestStmt) Pretty(int) string { return t.String() } func (t TestStmt) Pretty(int) string { return t.String() }
func (TestStmt) PositionRange() PositionRange { func (TestStmt) PositionRange() posrange.PositionRange {
return PositionRange{ return posrange.PositionRange{
Start: -1, Start: -1,
End: -1, End: -1,
} }
@ -405,17 +409,11 @@ func Children(node Node) []Node {
} }
} }
// PositionRange describes a position in the input string of the parser.
type PositionRange struct {
Start Pos
End Pos
}
// mergeRanges is a helper function to merge the PositionRanges of two Nodes. // mergeRanges is a helper function to merge the PositionRanges of two Nodes.
// Note that the arguments must be in the same order as they // Note that the arguments must be in the same order as they
// occur in the input string. // occur in the input string.
func mergeRanges(first, last Node) PositionRange { func mergeRanges(first, last Node) posrange.PositionRange {
return PositionRange{ return posrange.PositionRange{
Start: first.PositionRange().Start, Start: first.PositionRange().Start,
End: last.PositionRange().End, End: last.PositionRange().End,
} }
@ -423,33 +421,33 @@ func mergeRanges(first, last Node) PositionRange {
// Item implements the Node interface. // Item implements the Node interface.
// This makes it possible to call mergeRanges on them. // This makes it possible to call mergeRanges on them.
func (i *Item) PositionRange() PositionRange { func (i *Item) PositionRange() posrange.PositionRange {
return PositionRange{ return posrange.PositionRange{
Start: i.Pos, Start: i.Pos,
End: i.Pos + Pos(len(i.Val)), End: i.Pos + posrange.Pos(len(i.Val)),
} }
} }
func (e *AggregateExpr) PositionRange() PositionRange { func (e *AggregateExpr) PositionRange() posrange.PositionRange {
return e.PosRange return e.PosRange
} }
func (e *BinaryExpr) PositionRange() PositionRange { func (e *BinaryExpr) PositionRange() posrange.PositionRange {
return mergeRanges(e.LHS, e.RHS) return mergeRanges(e.LHS, e.RHS)
} }
func (e *Call) PositionRange() PositionRange { func (e *Call) PositionRange() posrange.PositionRange {
return e.PosRange return e.PosRange
} }
func (e *EvalStmt) PositionRange() PositionRange { func (e *EvalStmt) PositionRange() posrange.PositionRange {
return e.Expr.PositionRange() return e.Expr.PositionRange()
} }
func (e Expressions) PositionRange() PositionRange { func (e Expressions) PositionRange() posrange.PositionRange {
if len(e) == 0 { if len(e) == 0 {
// Position undefined. // Position undefined.
return PositionRange{ return posrange.PositionRange{
Start: -1, Start: -1,
End: -1, End: -1,
} }
@ -457,39 +455,39 @@ func (e Expressions) PositionRange() PositionRange {
return mergeRanges(e[0], e[len(e)-1]) return mergeRanges(e[0], e[len(e)-1])
} }
func (e *MatrixSelector) PositionRange() PositionRange { func (e *MatrixSelector) PositionRange() posrange.PositionRange {
return PositionRange{ return posrange.PositionRange{
Start: e.VectorSelector.PositionRange().Start, Start: e.VectorSelector.PositionRange().Start,
End: e.EndPos, End: e.EndPos,
} }
} }
func (e *SubqueryExpr) PositionRange() PositionRange { func (e *SubqueryExpr) PositionRange() posrange.PositionRange {
return PositionRange{ return posrange.PositionRange{
Start: e.Expr.PositionRange().Start, Start: e.Expr.PositionRange().Start,
End: e.EndPos, End: e.EndPos,
} }
} }
func (e *NumberLiteral) PositionRange() PositionRange { func (e *NumberLiteral) PositionRange() posrange.PositionRange {
return e.PosRange return e.PosRange
} }
func (e *ParenExpr) PositionRange() PositionRange { func (e *ParenExpr) PositionRange() posrange.PositionRange {
return e.PosRange return e.PosRange
} }
func (e *StringLiteral) PositionRange() PositionRange { func (e *StringLiteral) PositionRange() posrange.PositionRange {
return e.PosRange return e.PosRange
} }
func (e *UnaryExpr) PositionRange() PositionRange { func (e *UnaryExpr) PositionRange() posrange.PositionRange {
return PositionRange{ return posrange.PositionRange{
Start: e.StartPos, Start: e.StartPos,
End: e.Expr.PositionRange().End, End: e.Expr.PositionRange().End,
} }
} }
func (e *VectorSelector) PositionRange() PositionRange { func (e *VectorSelector) PositionRange() posrange.PositionRange {
return e.PosRange return e.PosRange
} }

View file

@ -22,6 +22,7 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/model/value" "github.com/prometheus/prometheus/model/value"
"github.com/prometheus/prometheus/model/histogram" "github.com/prometheus/prometheus/model/histogram"
"github.com/prometheus/prometheus/promql/parser/posrange"
) )
%} %}
@ -199,7 +200,7 @@ start :
{ yylex.(*parser).generatedParserResult = $2 } { yylex.(*parser).generatedParserResult = $2 }
| START_SERIES_DESCRIPTION series_description | START_SERIES_DESCRIPTION series_description
| START_EXPRESSION /* empty */ EOF | START_EXPRESSION /* empty */ EOF
{ yylex.(*parser).addParseErrf(PositionRange{}, "no expression found in input")} { yylex.(*parser).addParseErrf(posrange.PositionRange{}, "no expression found in input")}
| START_EXPRESSION expr | START_EXPRESSION expr
{ yylex.(*parser).generatedParserResult = $2 } { yylex.(*parser).generatedParserResult = $2 }
| START_METRIC_SELECTOR vector_selector | START_METRIC_SELECTOR vector_selector
@ -371,7 +372,7 @@ function_call : IDENTIFIER function_call_body
$$ = &Call{ $$ = &Call{
Func: fn, Func: fn,
Args: $2.(Expressions), Args: $2.(Expressions),
PosRange: PositionRange{ PosRange: posrange.PositionRange{
Start: $1.Pos, Start: $1.Pos,
End: yylex.(*parser).lastClosing, End: yylex.(*parser).lastClosing,
}, },

File diff suppressed because it is too large Load diff

View file

@ -19,13 +19,15 @@ import (
"strings" "strings"
"unicode" "unicode"
"unicode/utf8" "unicode/utf8"
"github.com/prometheus/prometheus/promql/parser/posrange"
) )
// Item represents a token or text string returned from the scanner. // Item represents a token or text string returned from the scanner.
type Item struct { type Item struct {
Typ ItemType // The type of this Item. Typ ItemType // The type of this Item.
Pos Pos // The starting position, in bytes, of this Item in the input string. Pos posrange.Pos // The starting position, in bytes, of this Item in the input string.
Val string // The value of this Item. Val string // The value of this Item.
} }
// String returns a descriptive string for the Item. // String returns a descriptive string for the Item.
@ -234,10 +236,6 @@ const eof = -1
// stateFn represents the state of the scanner as a function that returns the next state. // stateFn represents the state of the scanner as a function that returns the next state.
type stateFn func(*Lexer) stateFn type stateFn func(*Lexer) stateFn
// Pos is the position in a string.
// Negative numbers indicate undefined positions.
type Pos int
type histogramState int type histogramState int
const ( const (
@ -250,14 +248,14 @@ const (
// Lexer holds the state of the scanner. // Lexer holds the state of the scanner.
type Lexer struct { type Lexer struct {
input string // The string being scanned. input string // The string being scanned.
state stateFn // The next lexing function to enter. state stateFn // The next lexing function to enter.
pos Pos // Current position in the input. pos posrange.Pos // Current position in the input.
start Pos // Start position of this Item. start posrange.Pos // Start position of this Item.
width Pos // Width of last rune read from input. width posrange.Pos // Width of last rune read from input.
lastPos Pos // Position of most recent Item returned by NextItem. lastPos posrange.Pos // Position of most recent Item returned by NextItem.
itemp *Item // Pointer to where the next scanned item should be placed. itemp *Item // Pointer to where the next scanned item should be placed.
scannedItem bool // Set to true every time an item is scanned. scannedItem bool // Set to true every time an item is scanned.
parenDepth int // Nesting depth of ( ) exprs. parenDepth int // Nesting depth of ( ) exprs.
braceOpen bool // Whether a { is opened. braceOpen bool // Whether a { is opened.
@ -278,7 +276,7 @@ func (l *Lexer) next() rune {
return eof return eof
} }
r, w := utf8.DecodeRuneInString(l.input[l.pos:]) r, w := utf8.DecodeRuneInString(l.input[l.pos:])
l.width = Pos(w) l.width = posrange.Pos(w)
l.pos += l.width l.pos += l.width
return r return r
} }
@ -827,7 +825,7 @@ func lexSpace(l *Lexer) stateFn {
// lexLineComment scans a line comment. Left comment marker is known to be present. // lexLineComment scans a line comment. Left comment marker is known to be present.
func lexLineComment(l *Lexer) stateFn { func lexLineComment(l *Lexer) stateFn {
l.pos += Pos(len(lineComment)) l.pos += posrange.Pos(len(lineComment))
for r := l.next(); !isEndOfLine(r) && r != eof; { for r := l.next(); !isEndOfLine(r) && r != eof; {
r = l.next() r = l.next()
} }

View file

@ -17,6 +17,8 @@ import (
"testing" "testing"
"github.com/stretchr/testify/require" "github.com/stretchr/testify/require"
"github.com/prometheus/prometheus/promql/parser/posrange"
) )
type testCase struct { type testCase struct {
@ -824,7 +826,7 @@ func TestLexer(t *testing.T) {
require.Fail(t, "unexpected lexing error at position %d: %s", lastItem.Pos, lastItem) require.Fail(t, "unexpected lexing error at position %d: %s", lastItem.Pos, lastItem)
} }
eofItem := Item{EOF, Pos(len(test.input)), ""} eofItem := Item{EOF, posrange.Pos(len(test.input)), ""}
require.Equal(t, lastItem, eofItem, "%d: input %q", i, test.input) require.Equal(t, lastItem, eofItem, "%d: input %q", i, test.input)
out = out[:len(out)-1] out = out[:len(out)-1]

View file

@ -29,6 +29,7 @@ import (
"github.com/prometheus/prometheus/model/histogram" "github.com/prometheus/prometheus/model/histogram"
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/model/timestamp" "github.com/prometheus/prometheus/model/timestamp"
"github.com/prometheus/prometheus/promql/parser/posrange"
"github.com/prometheus/prometheus/util/strutil" "github.com/prometheus/prometheus/util/strutil"
) )
@ -54,7 +55,7 @@ type parser struct {
// Everytime an Item is lexed that could be the end // Everytime an Item is lexed that could be the end
// of certain expressions its end position is stored here. // of certain expressions its end position is stored here.
lastClosing Pos lastClosing posrange.Pos
yyParser yyParserImpl yyParser yyParserImpl
@ -121,7 +122,7 @@ func (p *parser) Close() {
// ParseErr wraps a parsing error with line and position context. // ParseErr wraps a parsing error with line and position context.
type ParseErr struct { type ParseErr struct {
PositionRange PositionRange PositionRange posrange.PositionRange
Err error Err error
Query string Query string
@ -130,27 +131,7 @@ type ParseErr struct {
} }
func (e *ParseErr) Error() string { func (e *ParseErr) Error() string {
pos := int(e.PositionRange.Start) return fmt.Sprintf("%s: parse error: %s", e.PositionRange.StartPosInput(e.Query, e.LineOffset), e.Err)
lastLineBreak := -1
line := e.LineOffset + 1
var positionStr string
if pos < 0 || pos > len(e.Query) {
positionStr = "invalid position:"
} else {
for i, c := range e.Query[:pos] {
if c == '\n' {
lastLineBreak = i
line++
}
}
col := pos - lastLineBreak
positionStr = fmt.Sprintf("%d:%d:", line, col)
}
return fmt.Sprintf("%s parse error: %s", positionStr, e.Err)
} }
type ParseErrors []ParseErr type ParseErrors []ParseErr
@ -275,12 +256,12 @@ func ParseSeriesDesc(input string) (labels labels.Labels, values []SequenceValue
} }
// addParseErrf formats the error and appends it to the list of parsing errors. // addParseErrf formats the error and appends it to the list of parsing errors.
func (p *parser) addParseErrf(positionRange PositionRange, format string, args ...interface{}) { func (p *parser) addParseErrf(positionRange posrange.PositionRange, format string, args ...interface{}) {
p.addParseErr(positionRange, fmt.Errorf(format, args...)) p.addParseErr(positionRange, fmt.Errorf(format, args...))
} }
// addParseErr appends the provided error to the list of parsing errors. // addParseErr appends the provided error to the list of parsing errors.
func (p *parser) addParseErr(positionRange PositionRange, err error) { func (p *parser) addParseErr(positionRange posrange.PositionRange, err error) {
perr := ParseErr{ perr := ParseErr{
PositionRange: positionRange, PositionRange: positionRange,
Err: err, Err: err,
@ -366,9 +347,9 @@ func (p *parser) Lex(lval *yySymType) int {
switch typ { switch typ {
case ERROR: case ERROR:
pos := PositionRange{ pos := posrange.PositionRange{
Start: p.lex.start, Start: p.lex.start,
End: Pos(len(p.lex.input)), End: posrange.Pos(len(p.lex.input)),
} }
p.addParseErr(pos, errors.New(p.yyParser.lval.item.Val)) p.addParseErr(pos, errors.New(p.yyParser.lval.item.Val))
@ -378,7 +359,7 @@ func (p *parser) Lex(lval *yySymType) int {
lval.item.Typ = EOF lval.item.Typ = EOF
p.InjectItem(0) p.InjectItem(0)
case RIGHT_BRACE, RIGHT_PAREN, RIGHT_BRACKET, DURATION, NUMBER: case RIGHT_BRACE, RIGHT_PAREN, RIGHT_BRACKET, DURATION, NUMBER:
p.lastClosing = lval.item.Pos + Pos(len(lval.item.Val)) p.lastClosing = lval.item.Pos + posrange.Pos(len(lval.item.Val))
} }
return int(typ) return int(typ)
@ -436,7 +417,7 @@ func (p *parser) newAggregateExpr(op Item, modifier, args Node) (ret *AggregateE
ret = modifier.(*AggregateExpr) ret = modifier.(*AggregateExpr)
arguments := args.(Expressions) arguments := args.(Expressions)
ret.PosRange = PositionRange{ ret.PosRange = posrange.PositionRange{
Start: op.Pos, Start: op.Pos,
End: p.lastClosing, End: p.lastClosing,
} }
@ -477,7 +458,7 @@ func (p *parser) newMap() (ret map[string]interface{}) {
func (p *parser) mergeMaps(left, right *map[string]interface{}) (ret *map[string]interface{}) { func (p *parser) mergeMaps(left, right *map[string]interface{}) (ret *map[string]interface{}) {
for key, value := range *right { for key, value := range *right {
if _, ok := (*left)[key]; ok { if _, ok := (*left)[key]; ok {
p.addParseErrf(PositionRange{}, "duplicate key \"%s\" in histogram", key) p.addParseErrf(posrange.PositionRange{}, "duplicate key \"%s\" in histogram", key)
continue continue
} }
(*left)[key] = value (*left)[key] = value
@ -677,7 +658,7 @@ func (p *parser) checkAST(node Node) (typ ValueType) {
// opRange returns the PositionRange of the operator part of the BinaryExpr. // opRange returns the PositionRange of the operator part of the BinaryExpr.
// This is made a function instead of a variable, so it is lazily evaluated on demand. // This is made a function instead of a variable, so it is lazily evaluated on demand.
opRange := func() (r PositionRange) { opRange := func() (r posrange.PositionRange) {
// Remove whitespace at the beginning and end of the range. // Remove whitespace at the beginning and end of the range.
for r.Start = n.LHS.PositionRange().End; isSpace(rune(p.lex.input[r.Start])); r.Start++ { // nolint:revive for r.Start = n.LHS.PositionRange().End; isSpace(rune(p.lex.input[r.Start])); r.Start++ { // nolint:revive
} }
@ -881,7 +862,7 @@ func (p *parser) newLabelMatcher(label, operator, value Item) *labels.Matcher {
// addOffset is used to set the offset in the generated parser. // addOffset is used to set the offset in the generated parser.
func (p *parser) addOffset(e Node, offset time.Duration) { func (p *parser) addOffset(e Node, offset time.Duration) {
var orgoffsetp *time.Duration var orgoffsetp *time.Duration
var endPosp *Pos var endPosp *posrange.Pos
switch s := e.(type) { switch s := e.(type) {
case *VectorSelector: case *VectorSelector:
@ -921,7 +902,7 @@ func (p *parser) setTimestamp(e Node, ts float64) {
p.addParseErrf(e.PositionRange(), "timestamp out of bounds for @ modifier: %f", ts) p.addParseErrf(e.PositionRange(), "timestamp out of bounds for @ modifier: %f", ts)
} }
var timestampp **int64 var timestampp **int64
var endPosp *Pos var endPosp *posrange.Pos
timestampp, _, endPosp, ok := p.getAtModifierVars(e) timestampp, _, endPosp, ok := p.getAtModifierVars(e)
if !ok { if !ok {
@ -950,11 +931,11 @@ func (p *parser) setAtModifierPreprocessor(e Node, op Item) {
*endPosp = p.lastClosing *endPosp = p.lastClosing
} }
func (p *parser) getAtModifierVars(e Node) (**int64, *ItemType, *Pos, bool) { func (p *parser) getAtModifierVars(e Node) (**int64, *ItemType, *posrange.Pos, bool) {
var ( var (
timestampp **int64 timestampp **int64
preprocp *ItemType preprocp *ItemType
endPosp *Pos endPosp *posrange.Pos
) )
switch s := e.(type) { switch s := e.(type) {
case *VectorSelector: case *VectorSelector:

File diff suppressed because it is too large Load diff

View file

@ -0,0 +1,54 @@
// Copyright 2023 The Prometheus Authors
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.
// posrange is used to report a position in query strings for error
// and warning messages.
package posrange
import "fmt"
// Pos is the position in a string.
// Negative numbers indicate undefined positions.
type Pos int
// PositionRange describes a position in the input string of the parser.
type PositionRange struct {
Start Pos
End Pos
}
// StartPosInput uses the query string to convert the PositionRange into a
// line:col string, indicating when this is not possible if the query is empty
// or the position is invalid. When this is used to convert ParseErr to a string,
// lineOffset is an additional line offset to be added, and is only used inside
// unit tests.
func (p PositionRange) StartPosInput(query string, lineOffset int) string {
if query == "" {
return "unknown position"
}
pos := int(p.Start)
if pos < 0 || pos > len(query) {
return "invalid position"
}
lastLineBreak := -1
line := lineOffset + 1
for i, c := range query[:pos] {
if c == '\n' {
lastLineBreak = i
line++
}
}
col := pos - lastLineBreak
return fmt.Sprintf("%d:%d", line, col)
}

View file

@ -34,6 +34,7 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/model/timestamp" "github.com/prometheus/prometheus/model/timestamp"
"github.com/prometheus/prometheus/promql/parser" "github.com/prometheus/prometheus/promql/parser"
"github.com/prometheus/prometheus/promql/parser/posrange"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/util/teststorage" "github.com/prometheus/prometheus/util/teststorage"
"github.com/prometheus/prometheus/util/testutil" "github.com/prometheus/prometheus/util/testutil"
@ -197,7 +198,7 @@ func (t *test) parseEval(lines []string, i int) (int, *evalCmd, error) {
if err != nil { if err != nil {
parser.EnrichParseError(err, func(parseErr *parser.ParseErr) { parser.EnrichParseError(err, func(parseErr *parser.ParseErr) {
parseErr.LineOffset = i parseErr.LineOffset = i
posOffset := parser.Pos(strings.Index(lines[i], expr)) posOffset := posrange.Pos(strings.Index(lines[i], expr))
parseErr.PositionRange.Start += posOffset parseErr.PositionRange.Start += posOffset
parseErr.PositionRange.End += posOffset parseErr.PositionRange.End += posOffset
parseErr.Query = lines[i] parseErr.Query = lines[i]

View file

@ -24,8 +24,8 @@ import (
"github.com/prometheus/prometheus/model/histogram" "github.com/prometheus/prometheus/model/histogram"
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/promql/parser" "github.com/prometheus/prometheus/promql/parser"
"github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/util/annotations"
) )
func (Matrix) Type() parser.ValueType { return parser.ValueTypeMatrix } func (Matrix) Type() parser.ValueType { return parser.ValueTypeMatrix }
@ -303,7 +303,7 @@ func (m Matrix) ContainsSameLabelset() bool {
type Result struct { type Result struct {
Err error Err error
Value parser.Value Value parser.Value
Warnings storage.Warnings Warnings annotations.Annotations
} }
// Vector returns a Vector if the result value is one. An error is returned if // Vector returns a Vector if the result value is one. An error is returned if

View file

@ -24,6 +24,7 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/util/annotations"
"github.com/prometheus/prometheus/util/teststorage" "github.com/prometheus/prometheus/util/teststorage"
) )
@ -178,8 +179,9 @@ func TestFanoutErrors(t *testing.T) {
if tc.warning != nil { if tc.warning != nil {
require.Greater(t, len(ss.Warnings()), 0, "warnings expected") require.Greater(t, len(ss.Warnings()), 0, "warnings expected")
require.Error(t, ss.Warnings()[0]) w := ss.Warnings()
require.Equal(t, tc.warning.Error(), ss.Warnings()[0].Error()) require.Error(t, w.AsErrors()[0])
require.Equal(t, tc.warning.Error(), w.AsStrings("", 0)[0])
} }
}) })
t.Run("chunks", func(t *testing.T) { t.Run("chunks", func(t *testing.T) {
@ -203,8 +205,9 @@ func TestFanoutErrors(t *testing.T) {
if tc.warning != nil { if tc.warning != nil {
require.Greater(t, len(ss.Warnings()), 0, "warnings expected") require.Greater(t, len(ss.Warnings()), 0, "warnings expected")
require.Error(t, ss.Warnings()[0]) w := ss.Warnings()
require.Equal(t, tc.warning.Error(), ss.Warnings()[0].Error()) require.Error(t, w.AsErrors()[0])
require.Equal(t, tc.warning.Error(), w.AsStrings("", 0)[0])
} }
}) })
} }
@ -233,11 +236,11 @@ func (errQuerier) Select(context.Context, bool, *storage.SelectHints, ...*labels
return storage.ErrSeriesSet(errSelect) return storage.ErrSeriesSet(errSelect)
} }
func (errQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (errQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, errors.New("label values error") return nil, nil, errors.New("label values error")
} }
func (errQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (errQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, errors.New("label names error") return nil, nil, errors.New("label names error")
} }

View file

@ -20,6 +20,7 @@ import (
"context" "context"
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/util/annotations"
) )
type genericQuerier interface { type genericQuerier interface {
@ -31,7 +32,7 @@ type genericSeriesSet interface {
Next() bool Next() bool
At() Labels At() Labels
Err() error Err() error
Warnings() Warnings Warnings() annotations.Annotations
} }
type genericSeriesMergeFunc func(...Labels) Labels type genericSeriesMergeFunc func(...Labels) Labels
@ -139,4 +140,4 @@ func (noopGenericSeriesSet) At() Labels { return nil }
func (noopGenericSeriesSet) Err() error { return nil } func (noopGenericSeriesSet) Err() error { return nil }
func (noopGenericSeriesSet) Warnings() Warnings { return nil } func (noopGenericSeriesSet) Warnings() annotations.Annotations { return nil }

View file

@ -24,6 +24,7 @@ import (
"github.com/prometheus/prometheus/model/metadata" "github.com/prometheus/prometheus/model/metadata"
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/tsdb/chunks" "github.com/prometheus/prometheus/tsdb/chunks"
"github.com/prometheus/prometheus/util/annotations"
) )
// The errors exposed. // The errors exposed.
@ -118,11 +119,11 @@ type MockQuerier struct {
SelectMockFunction func(sortSeries bool, hints *SelectHints, matchers ...*labels.Matcher) SeriesSet SelectMockFunction func(sortSeries bool, hints *SelectHints, matchers ...*labels.Matcher) SeriesSet
} }
func (q *MockQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, Warnings, error) { func (q *MockQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
func (q *MockQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, Warnings, error) { func (q *MockQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
@ -157,12 +158,12 @@ type LabelQuerier interface {
// It is not safe to use the strings beyond the lifetime of the querier. // It is not safe to use the strings beyond the lifetime of the querier.
// If matchers are specified the returned result set is reduced // If matchers are specified the returned result set is reduced
// to label values of metrics matching the matchers. // to label values of metrics matching the matchers.
LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, Warnings, error) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error)
// LabelNames returns all the unique label names present in the block in sorted order. // LabelNames returns all the unique label names present in the block in sorted order.
// If matchers are specified the returned result set is reduced // If matchers are specified the returned result set is reduced
// to label names of metrics matching the matchers. // to label names of metrics matching the matchers.
LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, Warnings, error) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error)
// Close releases the resources of the Querier. // Close releases the resources of the Querier.
Close() error Close() error
@ -307,7 +308,7 @@ type SeriesSet interface {
Err() error Err() error
// A collection of warnings for the whole set. // A collection of warnings for the whole set.
// Warnings could be return even iteration has not failed with error. // Warnings could be return even iteration has not failed with error.
Warnings() Warnings Warnings() annotations.Annotations
} }
var emptySeriesSet = errSeriesSet{} var emptySeriesSet = errSeriesSet{}
@ -321,10 +322,10 @@ type testSeriesSet struct {
series Series series Series
} }
func (s testSeriesSet) Next() bool { return true } func (s testSeriesSet) Next() bool { return true }
func (s testSeriesSet) At() Series { return s.series } func (s testSeriesSet) At() Series { return s.series }
func (s testSeriesSet) Err() error { return nil } func (s testSeriesSet) Err() error { return nil }
func (s testSeriesSet) Warnings() Warnings { return nil } func (s testSeriesSet) Warnings() annotations.Annotations { return nil }
// TestSeriesSet returns a mock series set // TestSeriesSet returns a mock series set
func TestSeriesSet(series Series) SeriesSet { func TestSeriesSet(series Series) SeriesSet {
@ -335,10 +336,10 @@ type errSeriesSet struct {
err error err error
} }
func (s errSeriesSet) Next() bool { return false } func (s errSeriesSet) Next() bool { return false }
func (s errSeriesSet) At() Series { return nil } func (s errSeriesSet) At() Series { return nil }
func (s errSeriesSet) Err() error { return s.err } func (s errSeriesSet) Err() error { return s.err }
func (s errSeriesSet) Warnings() Warnings { return nil } func (s errSeriesSet) Warnings() annotations.Annotations { return nil }
// ErrSeriesSet returns a series set that wraps an error. // ErrSeriesSet returns a series set that wraps an error.
func ErrSeriesSet(err error) SeriesSet { func ErrSeriesSet(err error) SeriesSet {
@ -356,10 +357,10 @@ type errChunkSeriesSet struct {
err error err error
} }
func (s errChunkSeriesSet) Next() bool { return false } func (s errChunkSeriesSet) Next() bool { return false }
func (s errChunkSeriesSet) At() ChunkSeries { return nil } func (s errChunkSeriesSet) At() ChunkSeries { return nil }
func (s errChunkSeriesSet) Err() error { return s.err } func (s errChunkSeriesSet) Err() error { return s.err }
func (s errChunkSeriesSet) Warnings() Warnings { return nil } func (s errChunkSeriesSet) Warnings() annotations.Annotations { return nil }
// ErrChunkSeriesSet returns a chunk series set that wraps an error. // ErrChunkSeriesSet returns a chunk series set that wraps an error.
func ErrChunkSeriesSet(err error) ChunkSeriesSet { func ErrChunkSeriesSet(err error) ChunkSeriesSet {
@ -405,7 +406,7 @@ type ChunkSeriesSet interface {
Err() error Err() error
// A collection of warnings for the whole set. // A collection of warnings for the whole set.
// Warnings could be return even iteration has not failed with error. // Warnings could be return even iteration has not failed with error.
Warnings() Warnings Warnings() annotations.Annotations
} }
// ChunkSeries exposes a single time series and allows iterating over chunks. // ChunkSeries exposes a single time series and allows iterating over chunks.
@ -433,5 +434,3 @@ type ChunkIterable interface {
// chunks of the series, sorted by min time. // chunks of the series, sorted by min time.
Iterator(chunks.Iterator) chunks.Iterator Iterator(chunks.Iterator) chunks.Iterator
} }
type Warnings []error

View file

@ -13,6 +13,10 @@
package storage package storage
import (
"github.com/prometheus/prometheus/util/annotations"
)
// lazyGenericSeriesSet is a wrapped series set that is initialised on first call to Next(). // lazyGenericSeriesSet is a wrapped series set that is initialised on first call to Next().
type lazyGenericSeriesSet struct { type lazyGenericSeriesSet struct {
init func() (genericSeriesSet, bool) init func() (genericSeriesSet, bool)
@ -43,25 +47,25 @@ func (c *lazyGenericSeriesSet) At() Labels {
return nil return nil
} }
func (c *lazyGenericSeriesSet) Warnings() Warnings { func (c *lazyGenericSeriesSet) Warnings() annotations.Annotations {
if c.set != nil { if c.set != nil {
return c.set.Warnings() return c.set.Warnings()
} }
return nil return nil
} }
type warningsOnlySeriesSet Warnings type warningsOnlySeriesSet annotations.Annotations
func (warningsOnlySeriesSet) Next() bool { return false } func (warningsOnlySeriesSet) Next() bool { return false }
func (warningsOnlySeriesSet) Err() error { return nil } func (warningsOnlySeriesSet) Err() error { return nil }
func (warningsOnlySeriesSet) At() Labels { return nil } func (warningsOnlySeriesSet) At() Labels { return nil }
func (c warningsOnlySeriesSet) Warnings() Warnings { return Warnings(c) } func (c warningsOnlySeriesSet) Warnings() annotations.Annotations { return annotations.Annotations(c) }
type errorOnlySeriesSet struct { type errorOnlySeriesSet struct {
err error err error
} }
func (errorOnlySeriesSet) Next() bool { return false } func (errorOnlySeriesSet) Next() bool { return false }
func (errorOnlySeriesSet) At() Labels { return nil } func (errorOnlySeriesSet) At() Labels { return nil }
func (s errorOnlySeriesSet) Err() error { return s.err } func (s errorOnlySeriesSet) Err() error { return s.err }
func (errorOnlySeriesSet) Warnings() Warnings { return nil } func (errorOnlySeriesSet) Warnings() annotations.Annotations { return nil }

View file

@ -28,6 +28,7 @@ import (
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/tsdb/chunks" "github.com/prometheus/prometheus/tsdb/chunks"
tsdb_errors "github.com/prometheus/prometheus/tsdb/errors" tsdb_errors "github.com/prometheus/prometheus/tsdb/errors"
"github.com/prometheus/prometheus/util/annotations"
) )
type mergeGenericQuerier struct { type mergeGenericQuerier struct {
@ -158,7 +159,7 @@ func (l labelGenericQueriers) SplitByHalf() (labelGenericQueriers, labelGenericQ
// LabelValues returns all potential values for a label name. // LabelValues returns all potential values for a label name.
// If matchers are specified the returned result set is reduced // If matchers are specified the returned result set is reduced
// to label values of metrics matching the matchers. // to label values of metrics matching the matchers.
func (q *mergeGenericQuerier) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, Warnings, error) { func (q *mergeGenericQuerier) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
res, ws, err := q.lvals(ctx, q.queriers, name, matchers...) res, ws, err := q.lvals(ctx, q.queriers, name, matchers...)
if err != nil { if err != nil {
return nil, nil, fmt.Errorf("LabelValues() from merge generic querier for label %s: %w", name, err) return nil, nil, fmt.Errorf("LabelValues() from merge generic querier for label %s: %w", name, err)
@ -167,7 +168,7 @@ func (q *mergeGenericQuerier) LabelValues(ctx context.Context, name string, matc
} }
// lvals performs merge sort for LabelValues from multiple queriers. // lvals performs merge sort for LabelValues from multiple queriers.
func (q *mergeGenericQuerier) lvals(ctx context.Context, lq labelGenericQueriers, n string, matchers ...*labels.Matcher) ([]string, Warnings, error) { func (q *mergeGenericQuerier) lvals(ctx context.Context, lq labelGenericQueriers, n string, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
if lq.Len() == 0 { if lq.Len() == 0 {
return nil, nil, nil return nil, nil, nil
} }
@ -176,14 +177,14 @@ func (q *mergeGenericQuerier) lvals(ctx context.Context, lq labelGenericQueriers
} }
a, b := lq.SplitByHalf() a, b := lq.SplitByHalf()
var ws Warnings var ws annotations.Annotations
s1, w, err := q.lvals(ctx, a, n, matchers...) s1, w, err := q.lvals(ctx, a, n, matchers...)
ws = append(ws, w...) ws.Merge(w)
if err != nil { if err != nil {
return nil, ws, err return nil, ws, err
} }
s2, ws, err := q.lvals(ctx, b, n, matchers...) s2, ws, err := q.lvals(ctx, b, n, matchers...)
ws = append(ws, w...) ws.Merge(w)
if err != nil { if err != nil {
return nil, ws, err return nil, ws, err
} }
@ -218,16 +219,16 @@ func mergeStrings(a, b []string) []string {
} }
// LabelNames returns all the unique label names present in all queriers in sorted order. // LabelNames returns all the unique label names present in all queriers in sorted order.
func (q *mergeGenericQuerier) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, Warnings, error) { func (q *mergeGenericQuerier) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
var ( var (
labelNamesMap = make(map[string]struct{}) labelNamesMap = make(map[string]struct{})
warnings Warnings warnings annotations.Annotations
) )
for _, querier := range q.queriers { for _, querier := range q.queriers {
names, wrn, err := querier.LabelNames(ctx, matchers...) names, wrn, err := querier.LabelNames(ctx, matchers...)
if wrn != nil { if wrn != nil {
// TODO(bwplotka): We could potentially wrap warnings. // TODO(bwplotka): We could potentially wrap warnings.
warnings = append(warnings, wrn...) warnings.Merge(wrn)
} }
if err != nil { if err != nil {
return nil, nil, fmt.Errorf("LabelNames() from merge generic querier: %w", err) return nil, nil, fmt.Errorf("LabelNames() from merge generic querier: %w", err)
@ -382,10 +383,10 @@ func (c *genericMergeSeriesSet) Err() error {
return nil return nil
} }
func (c *genericMergeSeriesSet) Warnings() Warnings { func (c *genericMergeSeriesSet) Warnings() annotations.Annotations {
var ws Warnings var ws annotations.Annotations
for _, set := range c.sets { for _, set := range c.sets {
ws = append(ws, set.Warnings()...) ws.Merge(set.Warnings())
} }
return ws return ws
} }

View file

@ -28,6 +28,7 @@ import (
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/tsdb/chunks" "github.com/prometheus/prometheus/tsdb/chunks"
"github.com/prometheus/prometheus/tsdb/tsdbutil" "github.com/prometheus/prometheus/tsdb/tsdbutil"
"github.com/prometheus/prometheus/util/annotations"
) )
func TestMergeQuerierWithChainMerger(t *testing.T) { func TestMergeQuerierWithChainMerger(t *testing.T) {
@ -777,7 +778,7 @@ func (m *mockSeriesSet) At() Series { return m.series[m.idx] }
func (m *mockSeriesSet) Err() error { return nil } func (m *mockSeriesSet) Err() error { return nil }
func (m *mockSeriesSet) Warnings() Warnings { return nil } func (m *mockSeriesSet) Warnings() annotations.Annotations { return nil }
type mockChunkSeriesSet struct { type mockChunkSeriesSet struct {
idx int idx int
@ -800,7 +801,7 @@ func (m *mockChunkSeriesSet) At() ChunkSeries { return m.series[m.idx] }
func (m *mockChunkSeriesSet) Err() error { return nil } func (m *mockChunkSeriesSet) Err() error { return nil }
func (m *mockChunkSeriesSet) Warnings() Warnings { return nil } func (m *mockChunkSeriesSet) Warnings() annotations.Annotations { return nil }
func TestChainSampleIterator(t *testing.T) { func TestChainSampleIterator(t *testing.T) {
for _, tc := range []struct { for _, tc := range []struct {
@ -974,7 +975,7 @@ type mockGenericQuerier struct {
sortedSeriesRequested []bool sortedSeriesRequested []bool
resp []string resp []string
warnings Warnings warnings annotations.Annotations
err error err error
} }
@ -990,7 +991,7 @@ func (m *mockGenericQuerier) Select(_ context.Context, b bool, _ *SelectHints, _
return &mockGenericSeriesSet{resp: m.resp, warnings: m.warnings, err: m.err} return &mockGenericSeriesSet{resp: m.resp, warnings: m.warnings, err: m.err}
} }
func (m *mockGenericQuerier) LabelValues(_ context.Context, name string, matchers ...*labels.Matcher) ([]string, Warnings, error) { func (m *mockGenericQuerier) LabelValues(_ context.Context, name string, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
m.mtx.Lock() m.mtx.Lock()
m.labelNamesRequested = append(m.labelNamesRequested, labelNameRequest{ m.labelNamesRequested = append(m.labelNamesRequested, labelNameRequest{
name: name, name: name,
@ -1000,7 +1001,7 @@ func (m *mockGenericQuerier) LabelValues(_ context.Context, name string, matcher
return m.resp, m.warnings, m.err return m.resp, m.warnings, m.err
} }
func (m *mockGenericQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, Warnings, error) { func (m *mockGenericQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
m.mtx.Lock() m.mtx.Lock()
m.labelNamesCalls++ m.labelNamesCalls++
m.mtx.Unlock() m.mtx.Unlock()
@ -1014,7 +1015,7 @@ func (m *mockGenericQuerier) Close() error {
type mockGenericSeriesSet struct { type mockGenericSeriesSet struct {
resp []string resp []string
warnings Warnings warnings annotations.Annotations
err error err error
curr int curr int
@ -1031,8 +1032,8 @@ func (m *mockGenericSeriesSet) Next() bool {
return true return true
} }
func (m *mockGenericSeriesSet) Err() error { return m.err } func (m *mockGenericSeriesSet) Err() error { return m.err }
func (m *mockGenericSeriesSet) Warnings() Warnings { return m.warnings } func (m *mockGenericSeriesSet) Warnings() annotations.Annotations { return m.warnings }
func (m *mockGenericSeriesSet) At() Labels { func (m *mockGenericSeriesSet) At() Labels {
return mockLabels(m.resp[m.curr-1]) return mockLabels(m.resp[m.curr-1])
@ -1068,10 +1069,9 @@ func TestMergeGenericQuerierWithSecondaries_ErrorHandling(t *testing.T) {
expectedSelectsSeries []labels.Labels expectedSelectsSeries []labels.Labels
expectedLabels []string expectedLabels []string
expectedWarnings [4]Warnings expectedWarnings annotations.Annotations
expectedErrs [4]error expectedErrs [4]error
}{ }{
{},
{ {
name: "one successful primary querier", name: "one successful primary querier",
queriers: []genericQuerier{&mockGenericQuerier{resp: []string{"a", "b"}, warnings: nil, err: nil}}, queriers: []genericQuerier{&mockGenericQuerier{resp: []string{"a", "b"}, warnings: nil, err: nil}},
@ -1145,31 +1145,21 @@ func TestMergeGenericQuerierWithSecondaries_ErrorHandling(t *testing.T) {
expectedSelectsSeries: []labels.Labels{ expectedSelectsSeries: []labels.Labels{
labels.FromStrings("test", "a"), labels.FromStrings("test", "a"),
}, },
expectedLabels: []string{"a"}, expectedLabels: []string{"a"},
expectedWarnings: [4]Warnings{ expectedWarnings: annotations.New().Add(errStorage),
[]error{errStorage, errStorage},
[]error{errStorage, errStorage},
[]error{errStorage, errStorage},
[]error{errStorage, errStorage},
},
}, },
{ {
name: "successful queriers with warnings", name: "successful queriers with warnings",
queriers: []genericQuerier{ queriers: []genericQuerier{
&mockGenericQuerier{resp: []string{"a"}, warnings: []error{warnStorage}, err: nil}, &mockGenericQuerier{resp: []string{"a"}, warnings: annotations.New().Add(warnStorage), err: nil},
&secondaryQuerier{genericQuerier: &mockGenericQuerier{resp: []string{"b"}, warnings: []error{warnStorage}, err: nil}}, &secondaryQuerier{genericQuerier: &mockGenericQuerier{resp: []string{"b"}, warnings: annotations.New().Add(warnStorage), err: nil}},
}, },
expectedSelectsSeries: []labels.Labels{ expectedSelectsSeries: []labels.Labels{
labels.FromStrings("test", "a"), labels.FromStrings("test", "a"),
labels.FromStrings("test", "b"), labels.FromStrings("test", "b"),
}, },
expectedLabels: []string{"a", "b"}, expectedLabels: []string{"a", "b"},
expectedWarnings: [4]Warnings{ expectedWarnings: annotations.New().Add(warnStorage),
[]error{warnStorage, warnStorage},
[]error{warnStorage, warnStorage},
[]error{warnStorage, warnStorage},
[]error{warnStorage, warnStorage},
},
}, },
} { } {
t.Run(tcase.name, func(t *testing.T) { t.Run(tcase.name, func(t *testing.T) {
@ -1184,7 +1174,7 @@ func TestMergeGenericQuerierWithSecondaries_ErrorHandling(t *testing.T) {
for res.Next() { for res.Next() {
lbls = append(lbls, res.At().Labels()) lbls = append(lbls, res.At().Labels())
} }
require.Equal(t, tcase.expectedWarnings[0], res.Warnings()) require.Subset(t, tcase.expectedWarnings, res.Warnings())
require.Equal(t, tcase.expectedErrs[0], res.Err()) require.Equal(t, tcase.expectedErrs[0], res.Err())
require.True(t, errors.Is(res.Err(), tcase.expectedErrs[0]), "expected error doesn't match") require.True(t, errors.Is(res.Err(), tcase.expectedErrs[0]), "expected error doesn't match")
require.Equal(t, tcase.expectedSelectsSeries, lbls) require.Equal(t, tcase.expectedSelectsSeries, lbls)
@ -1201,7 +1191,7 @@ func TestMergeGenericQuerierWithSecondaries_ErrorHandling(t *testing.T) {
}) })
t.Run("LabelNames", func(t *testing.T) { t.Run("LabelNames", func(t *testing.T) {
res, w, err := q.LabelNames(ctx) res, w, err := q.LabelNames(ctx)
require.Equal(t, tcase.expectedWarnings[1], w) require.Subset(t, tcase.expectedWarnings, w)
require.True(t, errors.Is(err, tcase.expectedErrs[1]), "expected error doesn't match") require.True(t, errors.Is(err, tcase.expectedErrs[1]), "expected error doesn't match")
require.Equal(t, tcase.expectedLabels, res) require.Equal(t, tcase.expectedLabels, res)
@ -1216,7 +1206,7 @@ func TestMergeGenericQuerierWithSecondaries_ErrorHandling(t *testing.T) {
}) })
t.Run("LabelValues", func(t *testing.T) { t.Run("LabelValues", func(t *testing.T) {
res, w, err := q.LabelValues(ctx, "test") res, w, err := q.LabelValues(ctx, "test")
require.Equal(t, tcase.expectedWarnings[2], w) require.Subset(t, tcase.expectedWarnings, w)
require.True(t, errors.Is(err, tcase.expectedErrs[2]), "expected error doesn't match") require.True(t, errors.Is(err, tcase.expectedErrs[2]), "expected error doesn't match")
require.Equal(t, tcase.expectedLabels, res) require.Equal(t, tcase.expectedLabels, res)
@ -1232,7 +1222,7 @@ func TestMergeGenericQuerierWithSecondaries_ErrorHandling(t *testing.T) {
t.Run("LabelValuesWithMatchers", func(t *testing.T) { t.Run("LabelValuesWithMatchers", func(t *testing.T) {
matcher := labels.MustNewMatcher(labels.MatchEqual, "otherLabel", "someValue") matcher := labels.MustNewMatcher(labels.MatchEqual, "otherLabel", "someValue")
res, w, err := q.LabelValues(ctx, "test2", matcher) res, w, err := q.LabelValues(ctx, "test2", matcher)
require.Equal(t, tcase.expectedWarnings[3], w) require.Subset(t, tcase.expectedWarnings, w)
require.True(t, errors.Is(err, tcase.expectedErrs[3]), "expected error doesn't match") require.True(t, errors.Is(err, tcase.expectedErrs[3]), "expected error doesn't match")
require.Equal(t, tcase.expectedLabels, res) require.Equal(t, tcase.expectedLabels, res)

View file

@ -17,6 +17,7 @@ import (
"context" "context"
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/util/annotations"
) )
type noopQuerier struct{} type noopQuerier struct{}
@ -30,11 +31,11 @@ func (noopQuerier) Select(context.Context, bool, *SelectHints, ...*labels.Matche
return NoopSeriesSet() return NoopSeriesSet()
} }
func (noopQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, Warnings, error) { func (noopQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
func (noopQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, Warnings, error) { func (noopQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
@ -53,11 +54,11 @@ func (noopChunkQuerier) Select(context.Context, bool, *SelectHints, ...*labels.M
return NoopChunkedSeriesSet() return NoopChunkedSeriesSet()
} }
func (noopChunkQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, Warnings, error) { func (noopChunkQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
func (noopChunkQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, Warnings, error) { func (noopChunkQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, nil return nil, nil, nil
} }
@ -78,7 +79,7 @@ func (noopSeriesSet) At() Series { return nil }
func (noopSeriesSet) Err() error { return nil } func (noopSeriesSet) Err() error { return nil }
func (noopSeriesSet) Warnings() Warnings { return nil } func (noopSeriesSet) Warnings() annotations.Annotations { return nil }
type noopChunkedSeriesSet struct{} type noopChunkedSeriesSet struct{}
@ -93,4 +94,4 @@ func (noopChunkedSeriesSet) At() ChunkSeries { return nil }
func (noopChunkedSeriesSet) Err() error { return nil } func (noopChunkedSeriesSet) Err() error { return nil }
func (noopChunkedSeriesSet) Warnings() Warnings { return nil } func (noopChunkedSeriesSet) Warnings() annotations.Annotations { return nil }

View file

@ -38,6 +38,7 @@ import (
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/tsdb/chunks" "github.com/prometheus/prometheus/tsdb/chunks"
"github.com/prometheus/prometheus/util/annotations"
) )
const ( const (
@ -122,7 +123,7 @@ func ToQuery(from, to int64, matchers []*labels.Matcher, hints *storage.SelectHi
} }
// ToQueryResult builds a QueryResult proto. // ToQueryResult builds a QueryResult proto.
func ToQueryResult(ss storage.SeriesSet, sampleLimit int) (*prompb.QueryResult, storage.Warnings, error) { func ToQueryResult(ss storage.SeriesSet, sampleLimit int) (*prompb.QueryResult, annotations.Annotations, error) {
numSamples := 0 numSamples := 0
resp := &prompb.QueryResult{} resp := &prompb.QueryResult{}
var iter chunkenc.Iterator var iter chunkenc.Iterator
@ -224,7 +225,7 @@ func StreamChunkedReadResponses(
sortedExternalLabels []prompb.Label, sortedExternalLabels []prompb.Label,
maxBytesInFrame int, maxBytesInFrame int,
marshalPool *sync.Pool, marshalPool *sync.Pool,
) (storage.Warnings, error) { ) (annotations.Annotations, error) {
var ( var (
chks []prompb.Chunk chks []prompb.Chunk
lbls []prompb.Label lbls []prompb.Label
@ -340,7 +341,7 @@ func (e errSeriesSet) Err() error {
return e.err return e.err
} }
func (e errSeriesSet) Warnings() storage.Warnings { return nil } func (e errSeriesSet) Warnings() annotations.Annotations { return nil }
// concreteSeriesSet implements storage.SeriesSet. // concreteSeriesSet implements storage.SeriesSet.
type concreteSeriesSet struct { type concreteSeriesSet struct {
@ -361,7 +362,7 @@ func (c *concreteSeriesSet) Err() error {
return nil return nil
} }
func (c *concreteSeriesSet) Warnings() storage.Warnings { return nil } func (c *concreteSeriesSet) Warnings() annotations.Annotations { return nil }
// concreteSeries implements storage.Series. // concreteSeries implements storage.Series.
type concreteSeries struct { type concreteSeries struct {

View file

@ -30,6 +30,7 @@ import (
"github.com/prometheus/prometheus/tsdb/chunkenc" "github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/tsdb/chunks" "github.com/prometheus/prometheus/tsdb/chunks"
"github.com/prometheus/prometheus/tsdb/tsdbutil" "github.com/prometheus/prometheus/tsdb/tsdbutil"
"github.com/prometheus/prometheus/util/annotations"
) )
var testHistogram = histogram.Histogram{ var testHistogram = histogram.Histogram{
@ -810,7 +811,7 @@ func (c *mockChunkSeriesSet) At() storage.ChunkSeries {
} }
} }
func (c *mockChunkSeriesSet) Warnings() storage.Warnings { return nil } func (c *mockChunkSeriesSet) Warnings() annotations.Annotations { return nil }
func (c *mockChunkSeriesSet) Err() error { func (c *mockChunkSeriesSet) Err() error {
return nil return nil

View file

@ -20,6 +20,7 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/util/annotations"
) )
type sampleAndChunkQueryableClient struct { type sampleAndChunkQueryableClient struct {
@ -209,13 +210,13 @@ func (q querier) addExternalLabels(ms []*labels.Matcher) ([]*labels.Matcher, []s
} }
// LabelValues implements storage.Querier and is a noop. // LabelValues implements storage.Querier and is a noop.
func (q *querier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (q *querier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
// TODO: Implement: https://github.com/prometheus/prometheus/issues/3351 // TODO: Implement: https://github.com/prometheus/prometheus/issues/3351
return nil, nil, errors.New("not implemented") return nil, nil, errors.New("not implemented")
} }
// LabelNames implements storage.Querier and is a noop. // LabelNames implements storage.Querier and is a noop.
func (q *querier) LabelNames(context.Context, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (q *querier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
// TODO: Implement: https://github.com/prometheus/prometheus/issues/3351 // TODO: Implement: https://github.com/prometheus/prometheus/issues/3351
return nil, nil, errors.New("not implemented") return nil, nil, errors.New("not implemented")
} }

View file

@ -27,6 +27,7 @@ import (
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/prompb" "github.com/prometheus/prometheus/prompb"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/util/annotations"
"github.com/prometheus/prometheus/util/gate" "github.com/prometheus/prometheus/util/gate"
) )
@ -154,7 +155,7 @@ func (h *readHandler) remoteReadSamples(
} }
} }
var ws storage.Warnings var ws annotations.Annotations
resp.Results[i], ws, err = ToQueryResult(querier.Select(ctx, false, hints, filteredMatchers...), h.remoteReadSampleLimit) resp.Results[i], ws, err = ToQueryResult(querier.Select(ctx, false, hints, filteredMatchers...), h.remoteReadSampleLimit)
if err != nil { if err != nil {
return err return err

View file

@ -27,7 +27,7 @@ import (
"github.com/prometheus/prometheus/config" "github.com/prometheus/prometheus/config"
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/prompb" "github.com/prometheus/prometheus/prompb"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/util/annotations"
) )
func TestNoDuplicateReadConfigs(t *testing.T) { func TestNoDuplicateReadConfigs(t *testing.T) {
@ -475,7 +475,7 @@ func TestSampleAndChunkQueryableClient(t *testing.T) {
ss := q.Select(context.Background(), true, nil, tc.matchers...) ss := q.Select(context.Background(), true, nil, tc.matchers...)
require.NoError(t, err) require.NoError(t, err)
require.Equal(t, storage.Warnings(nil), ss.Warnings()) require.Equal(t, annotations.Annotations(nil), ss.Warnings())
require.Equal(t, tc.expectedQuery, m.got) require.Equal(t, tc.expectedQuery, m.got)

View file

@ -18,6 +18,7 @@ import (
"sync" "sync"
"github.com/prometheus/prometheus/model/labels" "github.com/prometheus/prometheus/model/labels"
"github.com/prometheus/prometheus/util/annotations"
) )
// secondaryQuerier is a wrapper that allows a querier to be treated in a best effort manner. // secondaryQuerier is a wrapper that allows a querier to be treated in a best effort manner.
@ -48,18 +49,18 @@ func newSecondaryQuerierFromChunk(cq ChunkQuerier) genericQuerier {
return &secondaryQuerier{genericQuerier: newGenericQuerierFromChunk(cq)} return &secondaryQuerier{genericQuerier: newGenericQuerierFromChunk(cq)}
} }
func (s *secondaryQuerier) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, Warnings, error) { func (s *secondaryQuerier) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
vals, w, err := s.genericQuerier.LabelValues(ctx, name, matchers...) vals, w, err := s.genericQuerier.LabelValues(ctx, name, matchers...)
if err != nil { if err != nil {
return nil, append([]error{err}, w...), nil return nil, w.Add(err), nil
} }
return vals, w, nil return vals, w, nil
} }
func (s *secondaryQuerier) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, Warnings, error) { func (s *secondaryQuerier) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
names, w, err := s.genericQuerier.LabelNames(ctx, matchers...) names, w, err := s.genericQuerier.LabelNames(ctx, matchers...)
if err != nil { if err != nil {
return nil, append([]error{err}, w...), nil return nil, w.Add(err), nil
} }
return names, w, nil return names, w, nil
} }
@ -83,7 +84,7 @@ func (s *secondaryQuerier) Select(ctx context.Context, sortSeries bool, hints *S
if err := set.Err(); err != nil { if err := set.Err(); err != nil {
// One of the sets failed, ensure current one returning errors as warnings, and rest of the sets return nothing. // One of the sets failed, ensure current one returning errors as warnings, and rest of the sets return nothing.
// (All or nothing logic). // (All or nothing logic).
s.asyncSets[curr] = warningsOnlySeriesSet(append([]error{err}, ws...)) s.asyncSets[curr] = warningsOnlySeriesSet(ws.Add(err))
for i := range s.asyncSets { for i := range s.asyncSets {
if curr == i { if curr == i {
continue continue

View file

@ -53,6 +53,7 @@ import (
"github.com/prometheus/prometheus/tsdb/tombstones" "github.com/prometheus/prometheus/tsdb/tombstones"
"github.com/prometheus/prometheus/tsdb/tsdbutil" "github.com/prometheus/prometheus/tsdb/tsdbutil"
"github.com/prometheus/prometheus/tsdb/wlog" "github.com/prometheus/prometheus/tsdb/wlog"
"github.com/prometheus/prometheus/util/annotations"
"github.com/prometheus/prometheus/util/testutil" "github.com/prometheus/prometheus/util/testutil"
) )
@ -1731,7 +1732,7 @@ func TestNotMatcherSelectsLabelsUnsetSeries(t *testing.T) {
// expandSeriesSet returns the raw labels in the order they are retrieved from // expandSeriesSet returns the raw labels in the order they are retrieved from
// the series set and the samples keyed by Labels().String(). // the series set and the samples keyed by Labels().String().
func expandSeriesSet(ss storage.SeriesSet) ([]labels.Labels, map[string][]sample, storage.Warnings, error) { func expandSeriesSet(ss storage.SeriesSet) ([]labels.Labels, map[string][]sample, annotations.Annotations, error) {
resultLabels := []labels.Labels{} resultLabels := []labels.Labels{}
resultSamples := map[string][]sample{} resultSamples := map[string][]sample{}
var it chunkenc.Iterator var it chunkenc.Iterator
@ -1932,7 +1933,7 @@ func TestQuerierWithBoundaryChunks(t *testing.T) {
// The requested interval covers 2 blocks, so the querier's label values for blockID should give us 2 values, one from each block. // The requested interval covers 2 blocks, so the querier's label values for blockID should give us 2 values, one from each block.
b, ws, err := q.LabelValues(ctx, "blockID") b, ws, err := q.LabelValues(ctx, "blockID")
require.NoError(t, err) require.NoError(t, err)
require.Equal(t, storage.Warnings(nil), ws) require.Equal(t, annotations.Annotations{}, ws)
require.Equal(t, []string{"1", "2"}, b) require.Equal(t, []string{"1", "2"}, b)
} }
@ -2235,7 +2236,7 @@ func TestDB_LabelNames(t *testing.T) {
// Testing DB (union). // Testing DB (union).
q, err := db.Querier(math.MinInt64, math.MaxInt64) q, err := db.Querier(math.MinInt64, math.MaxInt64)
require.NoError(t, err) require.NoError(t, err)
var ws storage.Warnings var ws annotations.Annotations
labelNames, ws, err = q.LabelNames(ctx) labelNames, ws, err = q.LabelNames(ctx)
require.NoError(t, err) require.NoError(t, err)
require.Equal(t, 0, len(ws)) require.Equal(t, 0, len(ws))

View file

@ -32,6 +32,7 @@ import (
tsdb_errors "github.com/prometheus/prometheus/tsdb/errors" tsdb_errors "github.com/prometheus/prometheus/tsdb/errors"
"github.com/prometheus/prometheus/tsdb/index" "github.com/prometheus/prometheus/tsdb/index"
"github.com/prometheus/prometheus/tsdb/tombstones" "github.com/prometheus/prometheus/tsdb/tombstones"
"github.com/prometheus/prometheus/util/annotations"
) )
// Bitmap used by func isRegexMetaCharacter to check whether a character needs to be escaped. // Bitmap used by func isRegexMetaCharacter to check whether a character needs to be escaped.
@ -89,12 +90,12 @@ func newBlockBaseQuerier(b BlockReader, mint, maxt int64) (*blockBaseQuerier, er
}, nil }, nil
} }
func (q *blockBaseQuerier) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, storage.Warnings, error) { func (q *blockBaseQuerier) LabelValues(ctx context.Context, name string, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
res, err := q.index.SortedLabelValues(ctx, name, matchers...) res, err := q.index.SortedLabelValues(ctx, name, matchers...)
return res, nil, err return res, nil, err
} }
func (q *blockBaseQuerier) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, storage.Warnings, error) { func (q *blockBaseQuerier) LabelNames(ctx context.Context, matchers ...*labels.Matcher) ([]string, annotations.Annotations, error) {
res, err := q.index.LabelNames(ctx, matchers...) res, err := q.index.LabelNames(ctx, matchers...)
return res, nil, err return res, nil, err
} }
@ -615,7 +616,7 @@ func (b *blockBaseSeriesSet) Err() error {
return b.p.Err() return b.p.Err()
} }
func (b *blockBaseSeriesSet) Warnings() storage.Warnings { return nil } func (b *blockBaseSeriesSet) Warnings() annotations.Annotations { return nil }
// populateWithDelGenericSeriesIterator allows to iterate over given chunk // populateWithDelGenericSeriesIterator allows to iterate over given chunk
// metas. In each iteration it ensures that chunks are trimmed based on given // metas. In each iteration it ensures that chunks are trimmed based on given

View file

@ -38,20 +38,21 @@ import (
"github.com/prometheus/prometheus/tsdb/index" "github.com/prometheus/prometheus/tsdb/index"
"github.com/prometheus/prometheus/tsdb/tombstones" "github.com/prometheus/prometheus/tsdb/tombstones"
"github.com/prometheus/prometheus/tsdb/tsdbutil" "github.com/prometheus/prometheus/tsdb/tsdbutil"
"github.com/prometheus/prometheus/util/annotations"
) )
// TODO(bwplotka): Replace those mocks with remote.concreteSeriesSet. // TODO(bwplotka): Replace those mocks with remote.concreteSeriesSet.
type mockSeriesSet struct { type mockSeriesSet struct {
next func() bool next func() bool
series func() storage.Series series func() storage.Series
ws func() storage.Warnings ws func() annotations.Annotations
err func() error err func() error
} }
func (m *mockSeriesSet) Next() bool { return m.next() } func (m *mockSeriesSet) Next() bool { return m.next() }
func (m *mockSeriesSet) At() storage.Series { return m.series() } func (m *mockSeriesSet) At() storage.Series { return m.series() }
func (m *mockSeriesSet) Err() error { return m.err() } func (m *mockSeriesSet) Err() error { return m.err() }
func (m *mockSeriesSet) Warnings() storage.Warnings { return m.ws() } func (m *mockSeriesSet) Warnings() annotations.Annotations { return m.ws() }
func newMockSeriesSet(list []storage.Series) *mockSeriesSet { func newMockSeriesSet(list []storage.Series) *mockSeriesSet {
i := -1 i := -1
@ -64,21 +65,21 @@ func newMockSeriesSet(list []storage.Series) *mockSeriesSet {
return list[i] return list[i]
}, },
err: func() error { return nil }, err: func() error { return nil },
ws: func() storage.Warnings { return nil }, ws: func() annotations.Annotations { return nil },
} }
} }
type mockChunkSeriesSet struct { type mockChunkSeriesSet struct {
next func() bool next func() bool
series func() storage.ChunkSeries series func() storage.ChunkSeries
ws func() storage.Warnings ws func() annotations.Annotations
err func() error err func() error
} }
func (m *mockChunkSeriesSet) Next() bool { return m.next() } func (m *mockChunkSeriesSet) Next() bool { return m.next() }
func (m *mockChunkSeriesSet) At() storage.ChunkSeries { return m.series() } func (m *mockChunkSeriesSet) At() storage.ChunkSeries { return m.series() }
func (m *mockChunkSeriesSet) Err() error { return m.err() } func (m *mockChunkSeriesSet) Err() error { return m.err() }
func (m *mockChunkSeriesSet) Warnings() storage.Warnings { return m.ws() } func (m *mockChunkSeriesSet) Warnings() annotations.Annotations { return m.ws() }
func newMockChunkSeriesSet(list []storage.ChunkSeries) *mockChunkSeriesSet { func newMockChunkSeriesSet(list []storage.ChunkSeries) *mockChunkSeriesSet {
i := -1 i := -1
@ -91,7 +92,7 @@ func newMockChunkSeriesSet(list []storage.ChunkSeries) *mockChunkSeriesSet {
return list[i] return list[i]
}, },
err: func() error { return nil }, err: func() error { return nil },
ws: func() storage.Warnings { return nil }, ws: func() annotations.Annotations { return nil },
} }
} }

View file

@ -0,0 +1,165 @@
// Copyright 2023 The Prometheus Authors
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.
package annotations
import (
"errors"
"fmt"
"github.com/prometheus/common/model"
"github.com/prometheus/prometheus/promql/parser/posrange"
)
// Annotations is a general wrapper for warnings and other information
// that is returned by the query API along with the results.
// Each individual annotation is modeled by a Go error.
// They are deduplicated based on the string returned by error.Error().
// The zero value is usable without further initialization, see New().
type Annotations map[string]error
// New returns new Annotations ready to use. Note that the zero value of
// Annotations is also fully usable, but using this method is often more
// readable.
func New() *Annotations {
return &Annotations{}
}
// Add adds an annotation (modeled as a Go error) in-place and returns the
// modified Annotations for convenience.
func (a *Annotations) Add(err error) Annotations {
if *a == nil {
*a = Annotations{}
}
(*a)[err.Error()] = err
return *a
}
// Merge adds the contents of the second annotation to the first, modifying
// the first in-place, and returns the merged first Annotation for convenience.
func (a *Annotations) Merge(aa Annotations) Annotations {
if *a == nil {
*a = Annotations{}
}
for key, val := range aa {
(*a)[key] = val
}
return *a
}
// AsErrors is a convenience function to return the annotations map as a slice
// of errors.
func (a Annotations) AsErrors() []error {
arr := make([]error, 0, len(a))
for _, err := range a {
arr = append(arr, err)
}
return arr
}
// AsStrings is a convenience function to return the annotations map as a slice
// of strings. The query string is used to get the line number and character offset
// positioning info of the elements which trigger an annotation. We limit the number
// of annotations returned here with maxAnnos (0 for no limit).
func (a Annotations) AsStrings(query string, maxAnnos int) []string {
arr := make([]string, 0, len(a))
for _, err := range a {
if maxAnnos > 0 && len(arr) >= maxAnnos {
break
}
anErr, ok := err.(annoErr)
if ok {
anErr.Query = query
err = anErr
}
arr = append(arr, err.Error())
}
if maxAnnos > 0 && len(a) > maxAnnos {
arr = append(arr, fmt.Sprintf("%d more annotations omitted", len(a)-maxAnnos))
}
return arr
}
//nolint:revive // Ignore ST1012
var (
// Currently there are only 2 types, warnings and info.
// For now, info are visually identical with warnings as we have not updated
// the API spec or the frontend to show a different kind of warning. But we
// make the distinction here to prepare for adding them in future.
PromQLInfo = errors.New("PromQL info")
PromQLWarning = errors.New("PromQL warning")
InvalidQuantileWarning = fmt.Errorf("%w: quantile value should be between 0 and 1", PromQLWarning)
BadBucketLabelWarning = fmt.Errorf("%w: bucket label %q is missing or has a malformed value", PromQLWarning, model.BucketLabel)
MixedFloatsHistogramsWarning = fmt.Errorf("%w: encountered a mix of histograms and floats for metric name", PromQLWarning)
MixedClassicNativeHistogramsWarning = fmt.Errorf("%w: vector contains a mix of classic and native histograms for metric name", PromQLWarning)
PossibleNonCounterInfo = fmt.Errorf("%w: metric might not be a counter, name does not end in _total/_sum/_count:", PromQLInfo)
)
type annoErr struct {
PositionRange posrange.PositionRange
Err error
Query string
}
func (e annoErr) Error() string {
return fmt.Sprintf("%s (%s)", e.Err, e.PositionRange.StartPosInput(e.Query, 0))
}
// NewInvalidQuantileWarning is used when the user specifies an invalid quantile
// value, i.e. a float that is outside the range [0, 1] or NaN.
func NewInvalidQuantileWarning(q float64, pos posrange.PositionRange) annoErr {
return annoErr{
PositionRange: pos,
Err: fmt.Errorf("%w, got %g", InvalidQuantileWarning, q),
}
}
// NewBadBucketLabelWarning is used when there is an error parsing the bucket label
// of a classic histogram.
func NewBadBucketLabelWarning(metricName, label string, pos posrange.PositionRange) annoErr {
return annoErr{
PositionRange: pos,
Err: fmt.Errorf("%w of %q for metric name %q", BadBucketLabelWarning, label, metricName),
}
}
// NewMixedFloatsHistogramsWarning is used when the queried series includes both
// float samples and histogram samples for functions that do not support mixed
// samples.
func NewMixedFloatsHistogramsWarning(metricName string, pos posrange.PositionRange) annoErr {
return annoErr{
PositionRange: pos,
Err: fmt.Errorf("%w %q", MixedFloatsHistogramsWarning, metricName),
}
}
// NewMixedClassicNativeHistogramsWarning is used when the queried series includes
// both classic and native histograms.
func NewMixedClassicNativeHistogramsWarning(metricName string, pos posrange.PositionRange) annoErr {
return annoErr{
PositionRange: pos,
Err: fmt.Errorf("%w %q", MixedClassicNativeHistogramsWarning, metricName),
}
}
// NewPossibleNonCounterInfo is used when a counter metric does not have the suffixes
// _total, _sum or _count.
func NewPossibleNonCounterInfo(metricName string, pos posrange.PositionRange) annoErr {
return annoErr{
PositionRange: pos,
Err: fmt.Errorf("%w %q", PossibleNonCounterInfo, metricName),
}
}

View file

@ -51,6 +51,7 @@ import (
"github.com/prometheus/prometheus/storage/remote" "github.com/prometheus/prometheus/storage/remote"
"github.com/prometheus/prometheus/tsdb" "github.com/prometheus/prometheus/tsdb"
"github.com/prometheus/prometheus/tsdb/index" "github.com/prometheus/prometheus/tsdb/index"
"github.com/prometheus/prometheus/util/annotations"
"github.com/prometheus/prometheus/util/httputil" "github.com/prometheus/prometheus/util/httputil"
"github.com/prometheus/prometheus/util/stats" "github.com/prometheus/prometheus/util/stats"
) )
@ -161,7 +162,7 @@ type Response struct {
type apiFuncResult struct { type apiFuncResult struct {
data interface{} data interface{}
err *apiError err *apiError
warnings storage.Warnings warnings annotations.Annotations
finalizer func() finalizer func()
} }
@ -337,7 +338,7 @@ func (api *API) Register(r *route.Router) {
} }
if result.data != nil { if result.data != nil {
api.respond(w, r, result.data, result.warnings) api.respond(w, r, result.data, result.warnings, r.FormValue("query"))
return return
} }
w.WriteHeader(http.StatusNoContent) w.WriteHeader(http.StatusNoContent)
@ -667,7 +668,7 @@ func (api *API) labelNames(r *http.Request) apiFuncResult {
var ( var (
names []string names []string
warnings storage.Warnings warnings annotations.Annotations
) )
if len(matcherSets) > 0 { if len(matcherSets) > 0 {
labelNamesSet := make(map[string]struct{}) labelNamesSet := make(map[string]struct{})
@ -678,7 +679,7 @@ func (api *API) labelNames(r *http.Request) apiFuncResult {
return apiFuncResult{nil, returnAPIError(err), warnings, nil} return apiFuncResult{nil, returnAPIError(err), warnings, nil}
} }
warnings = append(warnings, callWarnings...) warnings.Merge(callWarnings)
for _, val := range vals { for _, val := range vals {
labelNamesSet[val] = struct{}{} labelNamesSet[val] = struct{}{}
} }
@ -743,17 +744,17 @@ func (api *API) labelValues(r *http.Request) (result apiFuncResult) {
var ( var (
vals []string vals []string
warnings storage.Warnings warnings annotations.Annotations
) )
if len(matcherSets) > 0 { if len(matcherSets) > 0 {
var callWarnings storage.Warnings var callWarnings annotations.Annotations
labelValuesSet := make(map[string]struct{}) labelValuesSet := make(map[string]struct{})
for _, matchers := range matcherSets { for _, matchers := range matcherSets {
vals, callWarnings, err = q.LabelValues(ctx, name, matchers...) vals, callWarnings, err = q.LabelValues(ctx, name, matchers...)
if err != nil { if err != nil {
return apiFuncResult{nil, &apiError{errorExec, err}, warnings, closer} return apiFuncResult{nil, &apiError{errorExec, err}, warnings, closer}
} }
warnings = append(warnings, callWarnings...) warnings.Merge(callWarnings)
for _, val := range vals { for _, val := range vals {
labelValuesSet[val] = struct{}{} labelValuesSet[val] = struct{}{}
} }
@ -1579,7 +1580,7 @@ func (api *API) serveWALReplayStatus(w http.ResponseWriter, r *http.Request) {
Min: status.Min, Min: status.Min,
Max: status.Max, Max: status.Max,
Current: status.Current, Current: status.Current,
}, nil) }, nil, "")
} }
func (api *API) remoteRead(w http.ResponseWriter, r *http.Request) { func (api *API) remoteRead(w http.ResponseWriter, r *http.Request) {
@ -1685,17 +1686,15 @@ func (api *API) cleanTombstones(*http.Request) apiFuncResult {
return apiFuncResult{nil, nil, nil, nil} return apiFuncResult{nil, nil, nil, nil}
} }
func (api *API) respond(w http.ResponseWriter, req *http.Request, data interface{}, warnings storage.Warnings) { // Query string is needed to get the position information for the annotations, and it
// can be empty if the position information isn't needed.
func (api *API) respond(w http.ResponseWriter, req *http.Request, data interface{}, warnings annotations.Annotations, query string) {
statusMessage := statusSuccess statusMessage := statusSuccess
var warningStrings []string
for _, warning := range warnings {
warningStrings = append(warningStrings, warning.Error())
}
resp := &Response{ resp := &Response{
Status: statusMessage, Status: statusMessage,
Data: data, Data: data,
Warnings: warningStrings, Warnings: warnings.AsStrings(query, 10),
} }
codec, err := api.negotiateCodec(req, resp) codec, err := api.negotiateCodec(req, resp)

View file

@ -2985,7 +2985,7 @@ func TestRespondSuccess(t *testing.T) {
api.InstallCodec(&testCodec{contentType: MIMEType{"test", "can-encode-2"}, canEncode: true}) api.InstallCodec(&testCodec{contentType: MIMEType{"test", "can-encode-2"}, canEncode: true})
s := httptest.NewServer(http.HandlerFunc(func(w http.ResponseWriter, r *http.Request) { s := httptest.NewServer(http.HandlerFunc(func(w http.ResponseWriter, r *http.Request) {
api.respond(w, r, "test", nil) api.respond(w, r, "test", nil, "")
})) }))
defer s.Close() defer s.Close()
@ -3074,7 +3074,7 @@ func TestRespondSuccess_DefaultCodecCannotEncodeResponse(t *testing.T) {
api.InstallCodec(&testCodec{contentType: MIMEType{"application", "default-format"}, canEncode: false}) api.InstallCodec(&testCodec{contentType: MIMEType{"application", "default-format"}, canEncode: false})
s := httptest.NewServer(http.HandlerFunc(func(w http.ResponseWriter, r *http.Request) { s := httptest.NewServer(http.HandlerFunc(func(w http.ResponseWriter, r *http.Request) {
api.respond(w, r, "test", nil) api.respond(w, r, "test", nil, "")
})) }))
defer s.Close() defer s.Close()
@ -3473,7 +3473,7 @@ func BenchmarkRespond(b *testing.B) {
api := API{} api := API{}
api.InstallCodec(JSONCodec{}) api.InstallCodec(JSONCodec{})
for n := 0; n < b.N; n++ { for n := 0; n < b.N; n++ {
api.respond(&testResponseWriter, request, c.response, nil) api.respond(&testResponseWriter, request, c.response, nil, "")
} }
}) })
} }

View file

@ -36,6 +36,7 @@ import (
"github.com/prometheus/prometheus/rules" "github.com/prometheus/prometheus/rules"
"github.com/prometheus/prometheus/scrape" "github.com/prometheus/prometheus/scrape"
"github.com/prometheus/prometheus/storage" "github.com/prometheus/prometheus/storage"
"github.com/prometheus/prometheus/util/annotations"
) )
func TestApiStatusCodes(t *testing.T) { func TestApiStatusCodes(t *testing.T) {
@ -170,11 +171,11 @@ type errorTestQuerier struct {
err error err error
} }
func (t errorTestQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (t errorTestQuerier) LabelValues(context.Context, string, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, t.err return nil, nil, t.err
} }
func (t errorTestQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, storage.Warnings, error) { func (t errorTestQuerier) LabelNames(context.Context, ...*labels.Matcher) ([]string, annotations.Annotations, error) {
return nil, nil, t.err return nil, nil, t.err
} }
@ -205,7 +206,7 @@ func (t errorTestSeriesSet) Err() error {
return t.err return t.err
} }
func (t errorTestSeriesSet) Warnings() storage.Warnings { func (t errorTestSeriesSet) Warnings() annotations.Annotations {
return nil return nil
} }