mirror of
https://github.com/prometheus/prometheus.git
synced 2025-01-12 14:27:27 -08:00
Merge pull request #5849 from csmarchbanks/rw-use-labels
Cache labels.Labels to Identify Series in Remote Write
This commit is contained in:
commit
b4317768b9
|
@ -131,7 +131,7 @@ func ToQueryResult(ss storage.SeriesSet, sampleLimit int) (*prompb.QueryResult,
|
||||||
}
|
}
|
||||||
|
|
||||||
resp.Timeseries = append(resp.Timeseries, &prompb.TimeSeries{
|
resp.Timeseries = append(resp.Timeseries, &prompb.TimeSeries{
|
||||||
Labels: labelsToLabelsProto(series.Labels()),
|
Labels: labelsToLabelsProto(series.Labels(), nil),
|
||||||
Samples: samples,
|
Samples: samples,
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
|
@ -192,6 +192,7 @@ func StreamChunkedReadResponses(
|
||||||
) error {
|
) error {
|
||||||
var (
|
var (
|
||||||
chks []prompb.Chunk
|
chks []prompb.Chunk
|
||||||
|
lbls []prompb.Label
|
||||||
err error
|
err error
|
||||||
lblsSize int
|
lblsSize int
|
||||||
)
|
)
|
||||||
|
@ -199,7 +200,7 @@ func StreamChunkedReadResponses(
|
||||||
for ss.Next() {
|
for ss.Next() {
|
||||||
series := ss.At()
|
series := ss.At()
|
||||||
iter := series.Iterator()
|
iter := series.Iterator()
|
||||||
lbls := MergeLabels(labelsToLabelsProto(series.Labels()), sortedExternalLabels)
|
lbls = MergeLabels(labelsToLabelsProto(series.Labels(), lbls), sortedExternalLabels)
|
||||||
|
|
||||||
lblsSize = 0
|
lblsSize = 0
|
||||||
for _, lbl := range lbls {
|
for _, lbl := range lbls {
|
||||||
|
@ -522,12 +523,17 @@ func labelProtosToLabels(labelPairs []prompb.Label) labels.Labels {
|
||||||
return result
|
return result
|
||||||
}
|
}
|
||||||
|
|
||||||
func labelsToLabelsProto(labels labels.Labels) []prompb.Label {
|
// labelsToLabelsProto transforms labels into prompb labels. The buffer slice
|
||||||
result := make([]prompb.Label, 0, len(labels))
|
// will be used to avoid allocations if it is big enough to store the labels.
|
||||||
|
func labelsToLabelsProto(labels labels.Labels, buf []prompb.Label) []prompb.Label {
|
||||||
|
result := buf[:0]
|
||||||
|
if cap(buf) < len(labels) {
|
||||||
|
result = make([]prompb.Label, 0, len(labels))
|
||||||
|
}
|
||||||
for _, l := range labels {
|
for _, l := range labels {
|
||||||
result = append(result, prompb.Label{
|
result = append(result, prompb.Label{
|
||||||
Name: interner.intern(l.Name),
|
Name: l.Name,
|
||||||
Value: interner.intern(l.Value),
|
Value: l.Value,
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
return result
|
return result
|
||||||
|
|
|
@ -166,7 +166,7 @@ type QueueManager struct {
|
||||||
client StorageClient
|
client StorageClient
|
||||||
watcher *WALWatcher
|
watcher *WALWatcher
|
||||||
|
|
||||||
seriesLabels map[uint64][]prompb.Label
|
seriesLabels map[uint64]labels.Labels
|
||||||
seriesSegmentIndexes map[uint64]int
|
seriesSegmentIndexes map[uint64]int
|
||||||
droppedSeries map[uint64]struct{}
|
droppedSeries map[uint64]struct{}
|
||||||
|
|
||||||
|
@ -208,7 +208,7 @@ func NewQueueManager(logger log.Logger, walDir string, samplesIn *ewmaRate, cfg
|
||||||
relabelConfigs: relabelConfigs,
|
relabelConfigs: relabelConfigs,
|
||||||
client: client,
|
client: client,
|
||||||
|
|
||||||
seriesLabels: make(map[uint64][]prompb.Label),
|
seriesLabels: make(map[uint64]labels.Labels),
|
||||||
seriesSegmentIndexes: make(map[uint64]int),
|
seriesSegmentIndexes: make(map[uint64]int),
|
||||||
droppedSeries: make(map[uint64]struct{}),
|
droppedSeries: make(map[uint64]struct{}),
|
||||||
|
|
||||||
|
@ -230,15 +230,15 @@ func NewQueueManager(logger log.Logger, walDir string, samplesIn *ewmaRate, cfg
|
||||||
|
|
||||||
// Append queues a sample to be sent to the remote storage. Blocks until all samples are
|
// Append queues a sample to be sent to the remote storage. Blocks until all samples are
|
||||||
// enqueued on their shards or a shutdown signal is received.
|
// enqueued on their shards or a shutdown signal is received.
|
||||||
func (t *QueueManager) Append(s []tsdb.RefSample) bool {
|
func (t *QueueManager) Append(samples []tsdb.RefSample) bool {
|
||||||
outer:
|
outer:
|
||||||
for _, sample := range s {
|
for _, s := range samples {
|
||||||
lbls, ok := t.seriesLabels[sample.Ref]
|
lbls, ok := t.seriesLabels[s.Ref]
|
||||||
if !ok {
|
if !ok {
|
||||||
t.droppedSamplesTotal.Inc()
|
t.droppedSamplesTotal.Inc()
|
||||||
t.samplesDropped.incr(1)
|
t.samplesDropped.incr(1)
|
||||||
if _, ok := t.droppedSeries[sample.Ref]; !ok {
|
if _, ok := t.droppedSeries[s.Ref]; !ok {
|
||||||
level.Info(t.logger).Log("msg", "dropped sample for series that was not explicitly dropped via relabelling", "ref", sample.Ref)
|
level.Info(t.logger).Log("msg", "dropped sample for series that was not explicitly dropped via relabelling", "ref", s.Ref)
|
||||||
}
|
}
|
||||||
continue
|
continue
|
||||||
}
|
}
|
||||||
|
@ -251,16 +251,11 @@ outer:
|
||||||
default:
|
default:
|
||||||
}
|
}
|
||||||
|
|
||||||
ts := prompb.TimeSeries{
|
if t.shards.enqueue(s.Ref, sample{
|
||||||
Labels: lbls,
|
labels: lbls,
|
||||||
Samples: []prompb.Sample{
|
t: s.T,
|
||||||
{
|
v: s.V,
|
||||||
Value: float64(sample.V),
|
}) {
|
||||||
Timestamp: sample.T,
|
|
||||||
},
|
|
||||||
},
|
|
||||||
}
|
|
||||||
if t.shards.enqueue(sample.Ref, ts) {
|
|
||||||
continue outer
|
continue outer
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -325,7 +320,7 @@ func (t *QueueManager) Stop() {
|
||||||
|
|
||||||
// On shutdown, release the strings in the labels from the intern pool.
|
// On shutdown, release the strings in the labels from the intern pool.
|
||||||
for _, labels := range t.seriesLabels {
|
for _, labels := range t.seriesLabels {
|
||||||
release(labels)
|
releaseLabels(labels)
|
||||||
}
|
}
|
||||||
// Delete metrics so we don't have alerts for queues that are gone.
|
// Delete metrics so we don't have alerts for queues that are gone.
|
||||||
name := t.client.Name()
|
name := t.client.Name()
|
||||||
|
@ -345,21 +340,21 @@ func (t *QueueManager) Stop() {
|
||||||
func (t *QueueManager) StoreSeries(series []tsdb.RefSeries, index int) {
|
func (t *QueueManager) StoreSeries(series []tsdb.RefSeries, index int) {
|
||||||
for _, s := range series {
|
for _, s := range series {
|
||||||
ls := processExternalLabels(s.Labels, t.externalLabels)
|
ls := processExternalLabels(s.Labels, t.externalLabels)
|
||||||
rl := relabel.Process(ls, t.relabelConfigs...)
|
lbls := relabel.Process(ls, t.relabelConfigs...)
|
||||||
if len(rl) == 0 {
|
if len(lbls) == 0 {
|
||||||
t.droppedSeries[s.Ref] = struct{}{}
|
t.droppedSeries[s.Ref] = struct{}{}
|
||||||
continue
|
continue
|
||||||
}
|
}
|
||||||
t.seriesSegmentIndexes[s.Ref] = index
|
t.seriesSegmentIndexes[s.Ref] = index
|
||||||
labels := labelsToLabelsProto(rl)
|
internLabels(lbls)
|
||||||
|
|
||||||
// We should not ever be replacing a series labels in the map, but just
|
// We should not ever be replacing a series labels in the map, but just
|
||||||
// in case we do we need to ensure we do not leak the replaced interned
|
// in case we do we need to ensure we do not leak the replaced interned
|
||||||
// strings.
|
// strings.
|
||||||
if orig, ok := t.seriesLabels[s.Ref]; ok {
|
if orig, ok := t.seriesLabels[s.Ref]; ok {
|
||||||
release(orig)
|
releaseLabels(orig)
|
||||||
}
|
}
|
||||||
t.seriesLabels[s.Ref] = labels
|
t.seriesLabels[s.Ref] = lbls
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -372,13 +367,20 @@ func (t *QueueManager) SeriesReset(index int) {
|
||||||
for k, v := range t.seriesSegmentIndexes {
|
for k, v := range t.seriesSegmentIndexes {
|
||||||
if v < index {
|
if v < index {
|
||||||
delete(t.seriesSegmentIndexes, k)
|
delete(t.seriesSegmentIndexes, k)
|
||||||
release(t.seriesLabels[k])
|
releaseLabels(t.seriesLabels[k])
|
||||||
delete(t.seriesLabels, k)
|
delete(t.seriesLabels, k)
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
func release(ls []prompb.Label) {
|
func internLabels(lbls labels.Labels) {
|
||||||
|
for i, l := range lbls {
|
||||||
|
lbls[i].Name = interner.intern(l.Name)
|
||||||
|
lbls[i].Value = interner.intern(l.Value)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
func releaseLabels(ls labels.Labels) {
|
||||||
for _, l := range ls {
|
for _, l := range ls {
|
||||||
interner.release(l.Name)
|
interner.release(l.Name)
|
||||||
interner.release(l.Value)
|
interner.release(l.Value)
|
||||||
|
@ -545,11 +547,17 @@ func (t *QueueManager) newShards() *shards {
|
||||||
return s
|
return s
|
||||||
}
|
}
|
||||||
|
|
||||||
|
type sample struct {
|
||||||
|
labels labels.Labels
|
||||||
|
t int64
|
||||||
|
v float64
|
||||||
|
}
|
||||||
|
|
||||||
type shards struct {
|
type shards struct {
|
||||||
mtx sync.RWMutex // With the WAL, this is never actually contended.
|
mtx sync.RWMutex // With the WAL, this is never actually contended.
|
||||||
|
|
||||||
qm *QueueManager
|
qm *QueueManager
|
||||||
queues []chan prompb.TimeSeries
|
queues []chan sample
|
||||||
|
|
||||||
// Emulate a wait group with a channel and an atomic int, as you
|
// Emulate a wait group with a channel and an atomic int, as you
|
||||||
// cannot select on a wait group.
|
// cannot select on a wait group.
|
||||||
|
@ -569,9 +577,9 @@ func (s *shards) start(n int) {
|
||||||
s.mtx.Lock()
|
s.mtx.Lock()
|
||||||
defer s.mtx.Unlock()
|
defer s.mtx.Unlock()
|
||||||
|
|
||||||
newQueues := make([]chan prompb.TimeSeries, n)
|
newQueues := make([]chan sample, n)
|
||||||
for i := 0; i < n; i++ {
|
for i := 0; i < n; i++ {
|
||||||
newQueues[i] = make(chan prompb.TimeSeries, s.qm.cfg.Capacity)
|
newQueues[i] = make(chan sample, s.qm.cfg.Capacity)
|
||||||
}
|
}
|
||||||
|
|
||||||
s.queues = newQueues
|
s.queues = newQueues
|
||||||
|
@ -619,7 +627,7 @@ func (s *shards) stop() {
|
||||||
|
|
||||||
// enqueue a sample. If we are currently in the process of shutting down or resharding,
|
// enqueue a sample. If we are currently in the process of shutting down or resharding,
|
||||||
// will return false; in this case, you should back off and retry.
|
// will return false; in this case, you should back off and retry.
|
||||||
func (s *shards) enqueue(ref uint64, sample prompb.TimeSeries) bool {
|
func (s *shards) enqueue(ref uint64, sample sample) bool {
|
||||||
s.mtx.RLock()
|
s.mtx.RLock()
|
||||||
defer s.mtx.RUnlock()
|
defer s.mtx.RUnlock()
|
||||||
|
|
||||||
|
@ -638,21 +646,24 @@ func (s *shards) enqueue(ref uint64, sample prompb.TimeSeries) bool {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
func (s *shards) runShard(ctx context.Context, i int, queue chan prompb.TimeSeries) {
|
func (s *shards) runShard(ctx context.Context, shardID int, queue chan sample) {
|
||||||
defer func() {
|
defer func() {
|
||||||
if atomic.AddInt32(&s.running, -1) == 0 {
|
if atomic.AddInt32(&s.running, -1) == 0 {
|
||||||
close(s.done)
|
close(s.done)
|
||||||
}
|
}
|
||||||
}()
|
}()
|
||||||
|
|
||||||
shardNum := strconv.Itoa(i)
|
shardNum := strconv.Itoa(shardID)
|
||||||
|
|
||||||
// Send batches of at most MaxSamplesPerSend samples to the remote storage.
|
// Send batches of at most MaxSamplesPerSend samples to the remote storage.
|
||||||
// If we have fewer samples than that, flush them out after a deadline
|
// If we have fewer samples than that, flush them out after a deadline
|
||||||
// anyways.
|
// anyways.
|
||||||
max := s.qm.cfg.MaxSamplesPerSend
|
var (
|
||||||
pendingSamples := make([]prompb.TimeSeries, 0, max)
|
max = s.qm.cfg.MaxSamplesPerSend
|
||||||
var buf []byte
|
nPending = 0
|
||||||
|
pendingSamples = allocateTimeSeries(max)
|
||||||
|
buf []byte
|
||||||
|
)
|
||||||
|
|
||||||
timer := time.NewTimer(time.Duration(s.qm.cfg.BatchSendDeadline))
|
timer := time.NewTimer(time.Duration(s.qm.cfg.BatchSendDeadline))
|
||||||
stop := func() {
|
stop := func() {
|
||||||
|
@ -672,24 +683,27 @@ func (s *shards) runShard(ctx context.Context, i int, queue chan prompb.TimeSeri
|
||||||
|
|
||||||
case sample, ok := <-queue:
|
case sample, ok := <-queue:
|
||||||
if !ok {
|
if !ok {
|
||||||
if len(pendingSamples) > 0 {
|
if nPending > 0 {
|
||||||
level.Debug(s.qm.logger).Log("msg", "Flushing samples to remote storage...", "count", len(pendingSamples))
|
level.Debug(s.qm.logger).Log("msg", "Flushing samples to remote storage...", "count", nPending)
|
||||||
s.sendSamples(ctx, pendingSamples, &buf)
|
s.sendSamples(ctx, pendingSamples[:nPending], &buf)
|
||||||
s.qm.pendingSamplesMetric.Sub(float64(len(pendingSamples)))
|
s.qm.pendingSamplesMetric.Sub(float64(nPending))
|
||||||
level.Debug(s.qm.logger).Log("msg", "Done flushing.")
|
level.Debug(s.qm.logger).Log("msg", "Done flushing.")
|
||||||
}
|
}
|
||||||
return
|
return
|
||||||
}
|
}
|
||||||
|
|
||||||
// Number of pending samples is limited by the fact that sendSamples (via sendSamplesWithBackoff)
|
// Number of pending samples is limited by the fact that sendSamples (via sendSamplesWithBackoff)
|
||||||
// retries endlessly, so once we reach > 100 samples, if we can never send to the endpoint we'll
|
// retries endlessly, so once we reach max samples, if we can never send to the endpoint we'll
|
||||||
// stop reading from the queue (which has a size of 10).
|
// stop reading from the queue. This makes it safe to reference pendingSamples by index.
|
||||||
pendingSamples = append(pendingSamples, sample)
|
pendingSamples[nPending].Labels = labelsToLabelsProto(sample.labels, pendingSamples[nPending].Labels)
|
||||||
|
pendingSamples[nPending].Samples[0].Timestamp = sample.t
|
||||||
|
pendingSamples[nPending].Samples[0].Value = sample.v
|
||||||
|
nPending++
|
||||||
s.qm.pendingSamplesMetric.Inc()
|
s.qm.pendingSamplesMetric.Inc()
|
||||||
|
|
||||||
if len(pendingSamples) >= max {
|
if nPending >= max {
|
||||||
s.sendSamples(ctx, pendingSamples[:max], &buf)
|
s.sendSamples(ctx, pendingSamples, &buf)
|
||||||
pendingSamples = append(pendingSamples[:0], pendingSamples[max:]...)
|
nPending = 0
|
||||||
s.qm.pendingSamplesMetric.Sub(float64(max))
|
s.qm.pendingSamplesMetric.Sub(float64(max))
|
||||||
|
|
||||||
stop()
|
stop()
|
||||||
|
@ -697,12 +711,11 @@ func (s *shards) runShard(ctx context.Context, i int, queue chan prompb.TimeSeri
|
||||||
}
|
}
|
||||||
|
|
||||||
case <-timer.C:
|
case <-timer.C:
|
||||||
n := len(pendingSamples)
|
if nPending > 0 {
|
||||||
if n > 0 {
|
level.Debug(s.qm.logger).Log("msg", "runShard timer ticked, sending samples", "samples", nPending, "shard", shardNum)
|
||||||
level.Debug(s.qm.logger).Log("msg", "runShard timer ticked, sending samples", "samples", n, "shard", shardNum)
|
s.sendSamples(ctx, pendingSamples[:nPending], &buf)
|
||||||
s.sendSamples(ctx, pendingSamples, &buf)
|
nPending = 0
|
||||||
pendingSamples = pendingSamples[:0]
|
s.qm.pendingSamplesMetric.Sub(float64(nPending))
|
||||||
s.qm.pendingSamplesMetric.Sub(float64(n))
|
|
||||||
}
|
}
|
||||||
timer.Reset(time.Duration(s.qm.cfg.BatchSendDeadline))
|
timer.Reset(time.Duration(s.qm.cfg.BatchSendDeadline))
|
||||||
}
|
}
|
||||||
|
@ -790,3 +803,12 @@ func buildWriteRequest(samples []prompb.TimeSeries, buf []byte) ([]byte, int64,
|
||||||
compressed := snappy.Encode(buf, data)
|
compressed := snappy.Encode(buf, data)
|
||||||
return compressed, highest, nil
|
return compressed, highest, nil
|
||||||
}
|
}
|
||||||
|
|
||||||
|
func allocateTimeSeries(capacity int) []prompb.TimeSeries {
|
||||||
|
timeseries := make([]prompb.TimeSeries, capacity)
|
||||||
|
// We only ever send one sample per timeseries, so preallocate with length one.
|
||||||
|
for i := range timeseries {
|
||||||
|
timeseries[i].Samples = []prompb.Sample{{}}
|
||||||
|
}
|
||||||
|
return timeseries
|
||||||
|
}
|
||||||
|
|
|
@ -133,12 +133,12 @@ func TestSeriesSetFilter(t *testing.T) {
|
||||||
toRemove: labels.Labels{{Name: "foo", Value: "bar"}},
|
toRemove: labels.Labels{{Name: "foo", Value: "bar"}},
|
||||||
in: &prompb.QueryResult{
|
in: &prompb.QueryResult{
|
||||||
Timeseries: []*prompb.TimeSeries{
|
Timeseries: []*prompb.TimeSeries{
|
||||||
{Labels: labelsToLabelsProto(labels.FromStrings("foo", "bar", "a", "b")), Samples: []prompb.Sample{}},
|
{Labels: labelsToLabelsProto(labels.FromStrings("foo", "bar", "a", "b"), nil), Samples: []prompb.Sample{}},
|
||||||
},
|
},
|
||||||
},
|
},
|
||||||
expected: &prompb.QueryResult{
|
expected: &prompb.QueryResult{
|
||||||
Timeseries: []*prompb.TimeSeries{
|
Timeseries: []*prompb.TimeSeries{
|
||||||
{Labels: labelsToLabelsProto(labels.FromStrings("a", "b")), Samples: []prompb.Sample{}},
|
{Labels: labelsToLabelsProto(labels.FromStrings("a", "b"), nil), Samples: []prompb.Sample{}},
|
||||||
},
|
},
|
||||||
},
|
},
|
||||||
},
|
},
|
||||||
|
|
Loading…
Reference in a new issue