2017-05-10 09:44:13 +00:00
|
|
|
// Copyright 2017 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
|
2017-10-17 01:26:38 +00:00
|
|
|
// limitations under the License.
|
2017-05-10 09:44:13 +00:00
|
|
|
|
|
|
|
package storage
|
|
|
|
|
|
|
|
import (
|
|
|
|
"container/heap"
|
2017-10-04 19:04:15 +00:00
|
|
|
"context"
|
2018-11-19 10:21:14 +00:00
|
|
|
"sort"
|
2017-05-10 09:44:13 +00:00
|
|
|
"strings"
|
|
|
|
|
2017-08-11 18:45:52 +00:00
|
|
|
"github.com/go-kit/kit/log"
|
|
|
|
"github.com/go-kit/kit/log/level"
|
2018-11-19 10:21:14 +00:00
|
|
|
"github.com/pkg/errors"
|
2017-10-18 11:08:14 +00:00
|
|
|
"github.com/prometheus/common/model"
|
2017-05-10 09:44:13 +00:00
|
|
|
"github.com/prometheus/prometheus/pkg/labels"
|
|
|
|
)
|
|
|
|
|
|
|
|
type fanout struct {
|
2017-08-11 18:45:52 +00:00
|
|
|
logger log.Logger
|
|
|
|
|
2017-07-12 14:50:26 +00:00
|
|
|
primary Storage
|
|
|
|
secondaries []Storage
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// NewFanout returns a new fan-out Storage, which proxies reads and writes
|
|
|
|
// through to multiple underlying storages.
|
2017-08-11 18:45:52 +00:00
|
|
|
func NewFanout(logger log.Logger, primary Storage, secondaries ...Storage) Storage {
|
2017-05-10 09:44:13 +00:00
|
|
|
return &fanout{
|
2017-08-11 18:45:52 +00:00
|
|
|
logger: logger,
|
2017-07-12 14:50:26 +00:00
|
|
|
primary: primary,
|
|
|
|
secondaries: secondaries,
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-10-18 11:08:14 +00:00
|
|
|
// StartTime implements the Storage interface.
|
|
|
|
func (f *fanout) StartTime() (int64, error) {
|
|
|
|
// StartTime of a fanout should be the earliest StartTime of all its storages,
|
|
|
|
// both primary and secondaries.
|
|
|
|
firstTime, err := f.primary.StartTime()
|
|
|
|
if err != nil {
|
|
|
|
return int64(model.Latest), err
|
|
|
|
}
|
|
|
|
|
|
|
|
for _, storage := range f.secondaries {
|
|
|
|
t, err := storage.StartTime()
|
|
|
|
if err != nil {
|
|
|
|
return int64(model.Latest), err
|
|
|
|
}
|
|
|
|
if t < firstTime {
|
|
|
|
firstTime = t
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return firstTime, nil
|
|
|
|
}
|
|
|
|
|
2017-10-04 19:04:15 +00:00
|
|
|
func (f *fanout) Querier(ctx context.Context, mint, maxt int64) (Querier, error) {
|
2017-10-27 11:29:05 +00:00
|
|
|
queriers := make([]Querier, 0, 1+len(f.secondaries))
|
2017-07-12 14:50:26 +00:00
|
|
|
|
|
|
|
// Add primary querier
|
2018-11-30 14:27:12 +00:00
|
|
|
primaryQuerier, err := f.primary.Querier(ctx, mint, maxt)
|
2017-07-12 14:50:26 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2018-11-30 14:27:12 +00:00
|
|
|
queriers = append(queriers, primaryQuerier)
|
2017-07-12 14:50:26 +00:00
|
|
|
|
|
|
|
// Add secondary queriers
|
|
|
|
for _, storage := range f.secondaries {
|
2017-10-04 19:04:15 +00:00
|
|
|
querier, err := storage.Querier(ctx, mint, maxt)
|
2017-05-10 09:44:13 +00:00
|
|
|
if err != nil {
|
2018-11-30 14:27:12 +00:00
|
|
|
NewMergeQuerier(primaryQuerier, queriers).Close()
|
2017-05-10 09:44:13 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
2017-10-27 11:29:05 +00:00
|
|
|
queriers = append(queriers, querier)
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
2017-10-27 11:29:05 +00:00
|
|
|
|
2018-11-30 14:27:12 +00:00
|
|
|
return NewMergeQuerier(primaryQuerier, queriers), nil
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
func (f *fanout) Appender() (Appender, error) {
|
2017-07-12 14:50:26 +00:00
|
|
|
primary, err := f.primary.Appender()
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
|
|
|
secondaries := make([]Appender, 0, len(f.secondaries))
|
|
|
|
for _, storage := range f.secondaries {
|
2017-05-10 09:44:13 +00:00
|
|
|
appender, err := storage.Appender()
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2017-07-12 14:50:26 +00:00
|
|
|
secondaries = append(secondaries, appender)
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
return &fanoutAppender{
|
2017-08-11 18:45:52 +00:00
|
|
|
logger: f.logger,
|
2017-07-12 14:50:26 +00:00
|
|
|
primary: primary,
|
|
|
|
secondaries: secondaries,
|
2017-05-10 09:44:13 +00:00
|
|
|
}, nil
|
|
|
|
}
|
|
|
|
|
|
|
|
// Close closes the storage and all its underlying resources.
|
|
|
|
func (f *fanout) Close() error {
|
2017-07-12 14:50:26 +00:00
|
|
|
if err := f.primary.Close(); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2017-05-10 09:44:13 +00:00
|
|
|
// TODO return multiple errors?
|
|
|
|
var lastErr error
|
2017-07-12 14:50:26 +00:00
|
|
|
for _, storage := range f.secondaries {
|
2017-05-10 09:44:13 +00:00
|
|
|
if err := storage.Close(); err != nil {
|
|
|
|
lastErr = err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return lastErr
|
|
|
|
}
|
|
|
|
|
|
|
|
// fanoutAppender implements Appender.
|
|
|
|
type fanoutAppender struct {
|
2017-08-11 18:45:52 +00:00
|
|
|
logger log.Logger
|
|
|
|
|
2017-07-12 14:50:26 +00:00
|
|
|
primary Appender
|
|
|
|
secondaries []Appender
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2017-09-07 12:14:41 +00:00
|
|
|
func (f *fanoutAppender) Add(l labels.Labels, t int64, v float64) (uint64, error) {
|
2017-07-12 14:50:26 +00:00
|
|
|
ref, err := f.primary.Add(l, t, v)
|
|
|
|
if err != nil {
|
|
|
|
return ref, err
|
|
|
|
}
|
|
|
|
|
|
|
|
for _, appender := range f.secondaries {
|
2017-05-10 09:44:13 +00:00
|
|
|
if _, err := appender.Add(l, t, v); err != nil {
|
2017-09-07 12:14:41 +00:00
|
|
|
return 0, err
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
}
|
2017-07-12 14:50:26 +00:00
|
|
|
return ref, nil
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2017-09-07 12:14:41 +00:00
|
|
|
func (f *fanoutAppender) AddFast(l labels.Labels, ref uint64, t int64, v float64) error {
|
2017-07-12 14:50:26 +00:00
|
|
|
if err := f.primary.AddFast(l, ref, t, v); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
|
|
|
for _, appender := range f.secondaries {
|
|
|
|
if _, err := appender.Add(l, t, v); err != nil {
|
2017-07-12 11:41:27 +00:00
|
|
|
return err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2017-07-12 14:50:26 +00:00
|
|
|
func (f *fanoutAppender) Commit() (err error) {
|
|
|
|
err = f.primary.Commit()
|
|
|
|
|
|
|
|
for _, appender := range f.secondaries {
|
|
|
|
if err == nil {
|
|
|
|
err = appender.Commit()
|
|
|
|
} else {
|
|
|
|
if rollbackErr := appender.Rollback(); rollbackErr != nil {
|
2017-08-11 18:45:52 +00:00
|
|
|
level.Error(f.logger).Log("msg", "Squashed rollback error on commit", "err", rollbackErr)
|
2017-07-12 14:50:26 +00:00
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
}
|
2017-07-12 14:50:26 +00:00
|
|
|
return
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2017-07-12 14:50:26 +00:00
|
|
|
func (f *fanoutAppender) Rollback() (err error) {
|
|
|
|
err = f.primary.Rollback()
|
|
|
|
|
|
|
|
for _, appender := range f.secondaries {
|
|
|
|
rollbackErr := appender.Rollback()
|
|
|
|
if err == nil {
|
|
|
|
err = rollbackErr
|
|
|
|
} else if rollbackErr != nil {
|
2017-08-11 18:45:52 +00:00
|
|
|
level.Error(f.logger).Log("msg", "Squashed rollback error on rollback", "err", rollbackErr)
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
// mergeQuerier implements Querier.
|
|
|
|
type mergeQuerier struct {
|
2018-11-30 14:27:12 +00:00
|
|
|
primaryQuerier Querier
|
|
|
|
queriers []Querier
|
|
|
|
|
|
|
|
failedQueriers map[Querier]struct{}
|
|
|
|
setQuerierMap map[SeriesSet]Querier
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2017-07-13 10:16:35 +00:00
|
|
|
// NewMergeQuerier returns a new Querier that merges results of input queriers.
|
2017-10-27 11:29:05 +00:00
|
|
|
// NB NewMergeQuerier will return NoopQuerier if no queriers are passed to it,
|
|
|
|
// and will filter NoopQueriers from its arguments, in order to reduce overhead
|
|
|
|
// when only one querier is passed.
|
2018-11-30 14:27:12 +00:00
|
|
|
func NewMergeQuerier(primaryQuerier Querier, queriers []Querier) Querier {
|
2017-10-27 11:29:05 +00:00
|
|
|
filtered := make([]Querier, 0, len(queriers))
|
|
|
|
for _, querier := range queriers {
|
|
|
|
if querier != NoopQuerier() {
|
|
|
|
filtered = append(filtered, querier)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-11-30 14:27:12 +00:00
|
|
|
setQuerierMap := make(map[SeriesSet]Querier)
|
|
|
|
failedQueriers := make(map[Querier]struct{})
|
|
|
|
|
2017-10-27 11:29:05 +00:00
|
|
|
switch len(filtered) {
|
|
|
|
case 0:
|
|
|
|
return NoopQuerier()
|
|
|
|
case 1:
|
|
|
|
return filtered[0]
|
|
|
|
default:
|
|
|
|
return &mergeQuerier{
|
2018-11-30 14:27:12 +00:00
|
|
|
primaryQuerier: primaryQuerier,
|
|
|
|
queriers: filtered,
|
|
|
|
failedQueriers: failedQueriers,
|
|
|
|
setQuerierMap: setQuerierMap,
|
2017-10-27 11:29:05 +00:00
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// Select returns a set of series that matches the given label matchers.
|
2019-01-02 11:10:13 +00:00
|
|
|
func (q *mergeQuerier) Select(params *SelectParams, matchers ...*labels.Matcher) (SeriesSet, Warnings, error) {
|
2017-05-10 09:44:13 +00:00
|
|
|
seriesSets := make([]SeriesSet, 0, len(q.queriers))
|
2018-11-30 14:27:12 +00:00
|
|
|
var warnings Warnings
|
2017-05-10 09:44:13 +00:00
|
|
|
for _, querier := range q.queriers {
|
2019-01-02 11:10:13 +00:00
|
|
|
set, wrn, err := querier.Select(params, matchers...)
|
2018-11-30 14:27:12 +00:00
|
|
|
q.setQuerierMap[set] = querier
|
|
|
|
if wrn != nil {
|
|
|
|
warnings = append(warnings, wrn...)
|
|
|
|
}
|
2017-11-23 12:50:06 +00:00
|
|
|
if err != nil {
|
2018-11-30 14:27:12 +00:00
|
|
|
q.failedQueriers[querier] = struct{}{}
|
|
|
|
// If the error source isn't the primary querier, return the error as a warning and continue.
|
|
|
|
if querier != q.primaryQuerier {
|
|
|
|
warnings = append(warnings, err)
|
|
|
|
continue
|
|
|
|
} else {
|
2019-01-02 11:10:13 +00:00
|
|
|
return nil, nil, err
|
2018-11-30 14:27:12 +00:00
|
|
|
}
|
2017-11-23 12:50:06 +00:00
|
|
|
}
|
|
|
|
seriesSets = append(seriesSets, set)
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
2019-01-02 11:10:13 +00:00
|
|
|
return NewMergeSeriesSet(seriesSets, q), warnings, nil
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// LabelValues returns all potential values for a label name.
|
2019-06-17 07:31:17 +00:00
|
|
|
func (q *mergeQuerier) LabelValues(name string) ([]string, Warnings, error) {
|
2017-05-10 09:44:13 +00:00
|
|
|
var results [][]string
|
2019-06-17 07:31:17 +00:00
|
|
|
var warnings Warnings
|
2017-05-10 09:44:13 +00:00
|
|
|
for _, querier := range q.queriers {
|
2019-06-17 07:31:17 +00:00
|
|
|
values, wrn, err := querier.LabelValues(name)
|
|
|
|
|
|
|
|
if wrn != nil {
|
|
|
|
warnings = append(warnings, wrn...)
|
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
if err != nil {
|
2019-06-17 07:31:17 +00:00
|
|
|
q.failedQueriers[querier] = struct{}{}
|
|
|
|
// If the error source isn't the primary querier, return the error as a warning and continue.
|
|
|
|
if querier != q.primaryQuerier {
|
|
|
|
warnings = append(warnings, err)
|
|
|
|
continue
|
|
|
|
} else {
|
|
|
|
return nil, nil, err
|
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
results = append(results, values)
|
|
|
|
}
|
2019-06-17 07:31:17 +00:00
|
|
|
return mergeStringSlices(results), warnings, nil
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2018-11-30 14:27:12 +00:00
|
|
|
func (q *mergeQuerier) IsFailedSet(set SeriesSet) bool {
|
|
|
|
_, isFailedQuerier := q.failedQueriers[q.setQuerierMap[set]]
|
|
|
|
return isFailedQuerier
|
|
|
|
}
|
|
|
|
|
2017-05-10 09:44:13 +00:00
|
|
|
func mergeStringSlices(ss [][]string) []string {
|
|
|
|
switch len(ss) {
|
|
|
|
case 0:
|
|
|
|
return nil
|
|
|
|
case 1:
|
|
|
|
return ss[0]
|
|
|
|
case 2:
|
|
|
|
return mergeTwoStringSlices(ss[0], ss[1])
|
|
|
|
default:
|
|
|
|
halfway := len(ss) / 2
|
|
|
|
return mergeTwoStringSlices(
|
|
|
|
mergeStringSlices(ss[:halfway]),
|
|
|
|
mergeStringSlices(ss[halfway:]),
|
|
|
|
)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func mergeTwoStringSlices(a, b []string) []string {
|
|
|
|
i, j := 0, 0
|
|
|
|
result := make([]string, 0, len(a)+len(b))
|
|
|
|
for i < len(a) && j < len(b) {
|
|
|
|
switch strings.Compare(a[i], b[j]) {
|
|
|
|
case 0:
|
|
|
|
result = append(result, a[i])
|
|
|
|
i++
|
|
|
|
j++
|
2017-07-13 10:05:38 +00:00
|
|
|
case -1:
|
2017-05-10 09:44:13 +00:00
|
|
|
result = append(result, a[i])
|
|
|
|
i++
|
2017-07-13 10:05:38 +00:00
|
|
|
case 1:
|
2017-05-10 09:44:13 +00:00
|
|
|
result = append(result, b[j])
|
|
|
|
j++
|
|
|
|
}
|
|
|
|
}
|
2017-07-13 10:05:38 +00:00
|
|
|
result = append(result, a[i:]...)
|
|
|
|
result = append(result, b[j:]...)
|
2017-05-10 09:44:13 +00:00
|
|
|
return result
|
|
|
|
}
|
|
|
|
|
2018-11-19 10:21:14 +00:00
|
|
|
// LabelNames returns all the unique label names present in the block in sorted order.
|
2019-06-17 07:31:17 +00:00
|
|
|
func (q *mergeQuerier) LabelNames() ([]string, Warnings, error) {
|
2018-11-19 10:21:14 +00:00
|
|
|
labelNamesMap := make(map[string]struct{})
|
2019-06-17 07:31:17 +00:00
|
|
|
var warnings Warnings
|
2018-11-19 10:21:14 +00:00
|
|
|
for _, b := range q.queriers {
|
2019-06-17 07:31:17 +00:00
|
|
|
names, wrn, err := b.LabelNames()
|
|
|
|
if wrn != nil {
|
|
|
|
warnings = append(warnings, wrn...)
|
|
|
|
}
|
|
|
|
|
2018-11-19 10:21:14 +00:00
|
|
|
if err != nil {
|
2019-06-17 07:31:17 +00:00
|
|
|
// If the error source isn't the primary querier, return the error as a warning and continue.
|
|
|
|
if b != q.primaryQuerier {
|
|
|
|
warnings = append(warnings, err)
|
|
|
|
continue
|
|
|
|
} else {
|
|
|
|
return nil, nil, errors.Wrap(err, "LabelNames() from Querier")
|
|
|
|
}
|
2018-11-19 10:21:14 +00:00
|
|
|
}
|
2019-06-17 07:31:17 +00:00
|
|
|
|
2018-11-19 10:21:14 +00:00
|
|
|
for _, name := range names {
|
|
|
|
labelNamesMap[name] = struct{}{}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
labelNames := make([]string, 0, len(labelNamesMap))
|
|
|
|
for name := range labelNamesMap {
|
|
|
|
labelNames = append(labelNames, name)
|
|
|
|
}
|
|
|
|
sort.Strings(labelNames)
|
|
|
|
|
2019-06-17 07:31:17 +00:00
|
|
|
return labelNames, warnings, nil
|
2018-11-19 10:21:14 +00:00
|
|
|
}
|
|
|
|
|
2017-05-10 09:44:13 +00:00
|
|
|
// Close releases the resources of the Querier.
|
|
|
|
func (q *mergeQuerier) Close() error {
|
|
|
|
// TODO return multiple errors?
|
|
|
|
var lastErr error
|
|
|
|
for _, querier := range q.queriers {
|
|
|
|
if err := querier.Close(); err != nil {
|
|
|
|
lastErr = err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return lastErr
|
|
|
|
}
|
|
|
|
|
|
|
|
// mergeSeriesSet implements SeriesSet
|
|
|
|
type mergeSeriesSet struct {
|
|
|
|
currentLabels labels.Labels
|
|
|
|
currentSets []SeriesSet
|
2017-07-13 14:02:01 +00:00
|
|
|
heap seriesSetHeap
|
|
|
|
sets []SeriesSet
|
2018-11-30 14:27:12 +00:00
|
|
|
|
|
|
|
querier *mergeQuerier
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2017-12-10 11:00:23 +00:00
|
|
|
// NewMergeSeriesSet returns a new series set that merges (deduplicates)
|
|
|
|
// series returned by the input series sets when iterating.
|
2018-11-30 14:27:12 +00:00
|
|
|
func NewMergeSeriesSet(sets []SeriesSet, querier *mergeQuerier) SeriesSet {
|
2018-01-25 11:17:16 +00:00
|
|
|
if len(sets) == 1 {
|
|
|
|
return sets[0]
|
|
|
|
}
|
|
|
|
|
2017-05-10 09:44:13 +00:00
|
|
|
// Sets need to be pre-advanced, so we can introspect the label of the
|
|
|
|
// series under the cursor.
|
|
|
|
var h seriesSetHeap
|
|
|
|
for _, set := range sets {
|
2018-11-30 14:27:12 +00:00
|
|
|
if set == nil {
|
|
|
|
continue
|
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
if set.Next() {
|
|
|
|
heap.Push(&h, set)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return &mergeSeriesSet{
|
2018-11-30 14:27:12 +00:00
|
|
|
heap: h,
|
|
|
|
sets: sets,
|
|
|
|
querier: querier,
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeSeriesSet) Next() bool {
|
2018-11-30 14:27:12 +00:00
|
|
|
// Run in a loop because the "next" series sets may not be valid anymore.
|
|
|
|
// If a remote querier fails, we discard all series sets from that querier.
|
|
|
|
// If, for the current label set, all the next series sets come from
|
|
|
|
// failed remote storage sources, we want to keep trying with the next label set.
|
|
|
|
for {
|
|
|
|
// Firstly advance all the current series sets. If any of them have run out
|
|
|
|
// we can drop them, otherwise they should be inserted back into the heap.
|
|
|
|
for _, set := range c.currentSets {
|
|
|
|
if set.Next() {
|
|
|
|
heap.Push(&c.heap, set)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
if len(c.heap) == 0 {
|
|
|
|
return false
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
2018-11-30 14:27:12 +00:00
|
|
|
// Now, pop items of the heap that have equal label sets.
|
|
|
|
c.currentSets = nil
|
|
|
|
c.currentLabels = c.heap[0].At().Labels()
|
|
|
|
for len(c.heap) > 0 && labels.Equal(c.currentLabels, c.heap[0].At().Labels()) {
|
|
|
|
set := heap.Pop(&c.heap).(SeriesSet)
|
|
|
|
if c.querier != nil && c.querier.IsFailedSet(set) {
|
|
|
|
continue
|
|
|
|
}
|
|
|
|
c.currentSets = append(c.currentSets, set)
|
|
|
|
}
|
|
|
|
|
|
|
|
// As long as the current set contains at least 1 set,
|
|
|
|
// then it should return true.
|
|
|
|
if len(c.currentSets) != 0 {
|
|
|
|
break
|
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
return true
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeSeriesSet) At() Series {
|
2018-01-25 11:11:55 +00:00
|
|
|
if len(c.currentSets) == 1 {
|
|
|
|
return c.currentSets[0].At()
|
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
series := []Series{}
|
|
|
|
for _, seriesSet := range c.currentSets {
|
|
|
|
series = append(series, seriesSet.At())
|
|
|
|
}
|
|
|
|
return &mergeSeries{
|
|
|
|
labels: c.currentLabels,
|
|
|
|
series: series,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeSeriesSet) Err() error {
|
|
|
|
for _, set := range c.sets {
|
|
|
|
if err := set.Err(); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
type seriesSetHeap []SeriesSet
|
|
|
|
|
|
|
|
func (h seriesSetHeap) Len() int { return len(h) }
|
|
|
|
func (h seriesSetHeap) Swap(i, j int) { h[i], h[j] = h[j], h[i] }
|
|
|
|
|
|
|
|
func (h seriesSetHeap) Less(i, j int) bool {
|
|
|
|
a, b := h[i].At().Labels(), h[j].At().Labels()
|
|
|
|
return labels.Compare(a, b) < 0
|
|
|
|
}
|
|
|
|
|
|
|
|
func (h *seriesSetHeap) Push(x interface{}) {
|
|
|
|
*h = append(*h, x.(SeriesSet))
|
|
|
|
}
|
|
|
|
|
|
|
|
func (h *seriesSetHeap) Pop() interface{} {
|
|
|
|
old := *h
|
|
|
|
n := len(old)
|
|
|
|
x := old[n-1]
|
|
|
|
*h = old[0 : n-1]
|
|
|
|
return x
|
|
|
|
}
|
|
|
|
|
|
|
|
type mergeSeries struct {
|
|
|
|
labels labels.Labels
|
|
|
|
series []Series
|
|
|
|
}
|
|
|
|
|
|
|
|
func (m *mergeSeries) Labels() labels.Labels {
|
|
|
|
return m.labels
|
|
|
|
}
|
|
|
|
|
|
|
|
func (m *mergeSeries) Iterator() SeriesIterator {
|
|
|
|
iterators := make([]SeriesIterator, 0, len(m.series))
|
|
|
|
for _, s := range m.series {
|
|
|
|
iterators = append(iterators, s.Iterator())
|
|
|
|
}
|
2017-07-13 13:40:29 +00:00
|
|
|
return newMergeIterator(iterators)
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
type mergeIterator struct {
|
|
|
|
iterators []SeriesIterator
|
|
|
|
h seriesIteratorHeap
|
|
|
|
}
|
|
|
|
|
|
|
|
func newMergeIterator(iterators []SeriesIterator) SeriesIterator {
|
|
|
|
return &mergeIterator{
|
|
|
|
iterators: iterators,
|
|
|
|
h: nil,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeIterator) Seek(t int64) bool {
|
|
|
|
c.h = seriesIteratorHeap{}
|
|
|
|
for _, iter := range c.iterators {
|
|
|
|
if iter.Seek(t) {
|
|
|
|
heap.Push(&c.h, iter)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return len(c.h) > 0
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeIterator) At() (t int64, v float64) {
|
2017-07-13 13:40:29 +00:00
|
|
|
if len(c.h) == 0 {
|
2017-08-11 18:45:52 +00:00
|
|
|
panic("mergeIterator.At() called after .Next() returned false.")
|
2017-07-13 13:40:29 +00:00
|
|
|
}
|
|
|
|
|
2017-05-10 09:44:13 +00:00
|
|
|
return c.h[0].At()
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeIterator) Next() bool {
|
|
|
|
if c.h == nil {
|
2017-07-13 10:05:38 +00:00
|
|
|
for _, iter := range c.iterators {
|
|
|
|
if iter.Next() {
|
|
|
|
heap.Push(&c.h, iter)
|
|
|
|
}
|
|
|
|
}
|
2018-05-23 11:15:47 +00:00
|
|
|
|
2017-07-13 10:05:38 +00:00
|
|
|
return len(c.h) > 0
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if len(c.h) == 0 {
|
|
|
|
return false
|
|
|
|
}
|
2017-07-13 10:05:38 +00:00
|
|
|
|
2018-06-18 16:34:08 +00:00
|
|
|
currt, _ := c.At()
|
2018-05-23 11:15:47 +00:00
|
|
|
for len(c.h) > 0 {
|
2018-06-18 16:34:08 +00:00
|
|
|
nextt, _ := c.h[0].At()
|
|
|
|
if nextt != currt {
|
2018-05-23 11:15:47 +00:00
|
|
|
break
|
|
|
|
}
|
|
|
|
|
|
|
|
iter := heap.Pop(&c.h).(SeriesIterator)
|
|
|
|
if iter.Next() {
|
|
|
|
heap.Push(&c.h, iter)
|
|
|
|
}
|
2017-05-10 09:44:13 +00:00
|
|
|
}
|
2017-07-13 10:05:38 +00:00
|
|
|
|
2017-05-10 09:44:13 +00:00
|
|
|
return len(c.h) > 0
|
|
|
|
}
|
|
|
|
|
|
|
|
func (c *mergeIterator) Err() error {
|
|
|
|
for _, iter := range c.iterators {
|
|
|
|
if err := iter.Err(); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
type seriesIteratorHeap []SeriesIterator
|
|
|
|
|
|
|
|
func (h seriesIteratorHeap) Len() int { return len(h) }
|
|
|
|
func (h seriesIteratorHeap) Swap(i, j int) { h[i], h[j] = h[j], h[i] }
|
|
|
|
|
|
|
|
func (h seriesIteratorHeap) Less(i, j int) bool {
|
|
|
|
at, _ := h[i].At()
|
|
|
|
bt, _ := h[j].At()
|
|
|
|
return at < bt
|
|
|
|
}
|
|
|
|
|
|
|
|
func (h *seriesIteratorHeap) Push(x interface{}) {
|
|
|
|
*h = append(*h, x.(SeriesIterator))
|
|
|
|
}
|
|
|
|
|
|
|
|
func (h *seriesIteratorHeap) Pop() interface{} {
|
|
|
|
old := *h
|
|
|
|
n := len(old)
|
|
|
|
x := old[n-1]
|
|
|
|
*h = old[0 : n-1]
|
|
|
|
return x
|
|
|
|
}
|