mirror of
https://github.com/prometheus/prometheus.git
synced 2024-12-24 13:14:05 -08:00
remote write 2.0 - follow up improvements (#13478)
* move remote write proto version config from a remote storage config to a per remote write configuration option Signed-off-by: Callum Styan <callumstyan@gmail.com> * rename scrape config for metadata, fix 2.0 header var name/value (was 1.1), and more clean up Signed-off-by: Callum Styan <callumstyan@gmail.com> * address review comments, mostly lint fixes Signed-off-by: Callum Styan <callumstyan@gmail.com> * another lint fix Signed-off-by: Callum Styan <callumstyan@gmail.com> * lint imports Signed-off-by: Callum Styan <callumstyan@gmail.com> --------- Signed-off-by: Callum Styan <callumstyan@gmail.com>
This commit is contained in:
parent
aa3513fc89
commit
552620a8ec
|
@ -193,7 +193,7 @@ func (c *flagConfig) setFeatureListOptions(logger log.Logger) error {
|
||||||
c.scrape.ExtraMetrics = true
|
c.scrape.ExtraMetrics = true
|
||||||
level.Info(logger).Log("msg", "Experimental additional scrape metrics enabled")
|
level.Info(logger).Log("msg", "Experimental additional scrape metrics enabled")
|
||||||
case "metadata-wal-records":
|
case "metadata-wal-records":
|
||||||
c.scrape.EnableMetadataStorage = true
|
c.scrape.AppendMetadata = true
|
||||||
level.Info(logger).Log("msg", "Experimental metadata records in WAL enabled, required for remote write 2.0")
|
level.Info(logger).Log("msg", "Experimental metadata records in WAL enabled, required for remote write 2.0")
|
||||||
case "new-service-discovery-manager":
|
case "new-service-discovery-manager":
|
||||||
c.enableNewSDManager = true
|
c.enableNewSDManager = true
|
||||||
|
@ -634,7 +634,7 @@ func main() {
|
||||||
var (
|
var (
|
||||||
localStorage = &readyStorage{stats: tsdb.NewDBStats()}
|
localStorage = &readyStorage{stats: tsdb.NewDBStats()}
|
||||||
scraper = &readyScrapeManager{}
|
scraper = &readyScrapeManager{}
|
||||||
remoteStorage = remote.NewStorage(log.With(logger, "component", "remote"), prometheus.DefaultRegisterer, localStorage.StartTime, localStoragePath, time.Duration(cfg.RemoteFlushDeadline), scraper, remote.RemoteWriteFormat(cfg.rwFormat), cfg.scrape.EnableMetadataStorage)
|
remoteStorage = remote.NewStorage(log.With(logger, "component", "remote"), prometheus.DefaultRegisterer, localStorage.StartTime, localStoragePath, time.Duration(cfg.RemoteFlushDeadline), scraper, cfg.scrape.AppendMetadata)
|
||||||
fanoutStorage = storage.NewFanout(logger, localStorage, remoteStorage)
|
fanoutStorage = storage.NewFanout(logger, localStorage, remoteStorage)
|
||||||
)
|
)
|
||||||
|
|
||||||
|
@ -819,7 +819,7 @@ func main() {
|
||||||
cfg.web.Flags[f.Name] = f.Value.String()
|
cfg.web.Flags[f.Name] = f.Value.String()
|
||||||
}
|
}
|
||||||
|
|
||||||
cfg.web.RemoteWriteFormat = remote.RemoteWriteFormat(cfg.rwFormat)
|
cfg.web.RemoteWriteFormat = config.RemoteWriteFormat(cfg.rwFormat)
|
||||||
// Depends on cfg.web.ScrapeManager so needs to be after cfg.web.ScrapeManager = scrapeManager.
|
// Depends on cfg.web.ScrapeManager so needs to be after cfg.web.ScrapeManager = scrapeManager.
|
||||||
webHandler := web.New(log.With(logger, "component", "web"), &cfg.web)
|
webHandler := web.New(log.With(logger, "component", "web"), &cfg.web)
|
||||||
|
|
||||||
|
|
|
@ -1016,6 +1016,9 @@ func CheckTargetAddress(address model.LabelValue) error {
|
||||||
return nil
|
return nil
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// This needs to live here rather than in the remote package to avoid an import cycle.
|
||||||
|
type RemoteWriteFormat int64
|
||||||
|
|
||||||
// RemoteWriteConfig is the configuration for writing to remote storage.
|
// RemoteWriteConfig is the configuration for writing to remote storage.
|
||||||
type RemoteWriteConfig struct {
|
type RemoteWriteConfig struct {
|
||||||
URL *config.URL `yaml:"url"`
|
URL *config.URL `yaml:"url"`
|
||||||
|
@ -1025,6 +1028,7 @@ type RemoteWriteConfig struct {
|
||||||
Name string `yaml:"name,omitempty"`
|
Name string `yaml:"name,omitempty"`
|
||||||
SendExemplars bool `yaml:"send_exemplars,omitempty"`
|
SendExemplars bool `yaml:"send_exemplars,omitempty"`
|
||||||
SendNativeHistograms bool `yaml:"send_native_histograms,omitempty"`
|
SendNativeHistograms bool `yaml:"send_native_histograms,omitempty"`
|
||||||
|
ProtocolVersion RemoteWriteFormat `yaml:"remote_write_version,omitempty"`
|
||||||
|
|
||||||
// We cannot do proper Go type embedding below as the parser will then parse
|
// We cannot do proper Go type embedding below as the parser will then parse
|
||||||
// values arbitrarily into the overflow maps of further-down types.
|
// values arbitrarily into the overflow maps of further-down types.
|
||||||
|
|
|
@ -73,9 +73,11 @@ type Options struct {
|
||||||
// Option used by downstream scraper users like OpenTelemetry Collector
|
// Option used by downstream scraper users like OpenTelemetry Collector
|
||||||
// to help lookup metric metadata. Should be false for Prometheus.
|
// to help lookup metric metadata. Should be false for Prometheus.
|
||||||
PassMetadataInContext bool
|
PassMetadataInContext bool
|
||||||
// Option to enable the experimental in-memory metadata storage and append
|
// Option to enable appending of scraped Metadata to the TSDB/other appenders. Individual appenders
|
||||||
// metadata to the WAL.
|
// can decide what to do with metadata, but for practical purposes this flag exists so that metadata
|
||||||
EnableMetadataStorage bool
|
// can be written to the WAL and thus read for remote write.
|
||||||
|
// TODO: implement some form of metadata storage
|
||||||
|
AppendMetadata bool
|
||||||
// Option to increase the interval used by scrape manager to throttle target groups updates.
|
// Option to increase the interval used by scrape manager to throttle target groups updates.
|
||||||
DiscoveryReloadInterval model.Duration
|
DiscoveryReloadInterval model.Duration
|
||||||
// Option to enable the ingestion of the created timestamp as a synthetic zero sample.
|
// Option to enable the ingestion of the created timestamp as a synthetic zero sample.
|
||||||
|
|
|
@ -173,7 +173,7 @@ func newScrapePool(cfg *config.ScrapeConfig, app storage.Appendable, offsetSeed
|
||||||
opts.scrapeClassicHistograms,
|
opts.scrapeClassicHistograms,
|
||||||
options.EnableCreatedTimestampZeroIngestion,
|
options.EnableCreatedTimestampZeroIngestion,
|
||||||
options.ExtraMetrics,
|
options.ExtraMetrics,
|
||||||
options.EnableMetadataStorage,
|
options.AppendMetadata,
|
||||||
opts.target,
|
opts.target,
|
||||||
options.PassMetadataInContext,
|
options.PassMetadataInContext,
|
||||||
metrics,
|
metrics,
|
||||||
|
|
|
@ -32,10 +32,12 @@ import (
|
||||||
"github.com/prometheus/common/model"
|
"github.com/prometheus/common/model"
|
||||||
"github.com/prometheus/common/sigv4"
|
"github.com/prometheus/common/sigv4"
|
||||||
"github.com/prometheus/common/version"
|
"github.com/prometheus/common/version"
|
||||||
|
|
||||||
"go.opentelemetry.io/contrib/instrumentation/net/http/otelhttp"
|
"go.opentelemetry.io/contrib/instrumentation/net/http/otelhttp"
|
||||||
"go.opentelemetry.io/otel"
|
"go.opentelemetry.io/otel"
|
||||||
"go.opentelemetry.io/otel/trace"
|
"go.opentelemetry.io/otel/trace"
|
||||||
|
|
||||||
|
"github.com/prometheus/prometheus/config"
|
||||||
"github.com/prometheus/prometheus/prompb"
|
"github.com/prometheus/prometheus/prompb"
|
||||||
"github.com/prometheus/prometheus/storage/remote/azuread"
|
"github.com/prometheus/prometheus/storage/remote/azuread"
|
||||||
)
|
)
|
||||||
|
@ -83,7 +85,7 @@ func init() {
|
||||||
type Client struct {
|
type Client struct {
|
||||||
remoteName string // Used to differentiate clients in metrics.
|
remoteName string // Used to differentiate clients in metrics.
|
||||||
urlString string // url.String()
|
urlString string // url.String()
|
||||||
rwFormat RemoteWriteFormat // For write clients, ignored for read clients.
|
rwFormat config.RemoteWriteFormat // For write clients, ignored for read clients.
|
||||||
Client *http.Client
|
Client *http.Client
|
||||||
timeout time.Duration
|
timeout time.Duration
|
||||||
|
|
||||||
|
@ -97,7 +99,7 @@ type Client struct {
|
||||||
// ClientConfig configures a client.
|
// ClientConfig configures a client.
|
||||||
type ClientConfig struct {
|
type ClientConfig struct {
|
||||||
URL *config_util.URL
|
URL *config_util.URL
|
||||||
RemoteWriteFormat RemoteWriteFormat
|
RemoteWriteFormat config.RemoteWriteFormat
|
||||||
Timeout model.Duration
|
Timeout model.Duration
|
||||||
HTTPClientConfig config_util.HTTPClientConfig
|
HTTPClientConfig config_util.HTTPClientConfig
|
||||||
SigV4Config *sigv4.SigV4Config
|
SigV4Config *sigv4.SigV4Config
|
||||||
|
@ -215,7 +217,7 @@ func (c *Client) Store(ctx context.Context, req []byte, attempt int) error {
|
||||||
httpReq.Header.Set(RemoteWriteVersionHeader, RemoteWriteVersion1HeaderValue)
|
httpReq.Header.Set(RemoteWriteVersionHeader, RemoteWriteVersion1HeaderValue)
|
||||||
} else {
|
} else {
|
||||||
// Set the right header if we're using v1.1 remote write protocol
|
// Set the right header if we're using v1.1 remote write protocol
|
||||||
httpReq.Header.Set(RemoteWriteVersionHeader, RemoteWriteVersion11HeaderValue)
|
httpReq.Header.Set(RemoteWriteVersionHeader, RemoteWriteVersion2HeaderValue)
|
||||||
}
|
}
|
||||||
|
|
||||||
if attempt > 0 {
|
if attempt > 0 {
|
||||||
|
|
|
@ -568,8 +568,7 @@ func TestDecodeWriteRequest(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestDecodeMinWriteRequest(t *testing.T) {
|
func TestDecodeMinWriteRequest(t *testing.T) {
|
||||||
buf, _, _, err := buildMinimizedWriteRequestStr(log.NewNopLogger(), writeRequestMinimizedFixture.Timeseries, writeRequestMinimizedFixture.Symbols, nil, nil, nil)
|
buf, _, _, err := buildV2WriteRequest(log.NewNopLogger(), writeRequestMinimizedFixture.Timeseries, writeRequestMinimizedFixture.Symbols, nil, nil, nil)
|
||||||
|
|
||||||
require.NoError(t, err)
|
require.NoError(t, err)
|
||||||
|
|
||||||
actual, err := DecodeMinimizedWriteRequestStr(bytes.NewReader(buf))
|
actual, err := DecodeMinimizedWriteRequestStr(bytes.NewReader(buf))
|
||||||
|
|
|
@ -394,10 +394,8 @@ type WriteClient interface {
|
||||||
Endpoint() string
|
Endpoint() string
|
||||||
}
|
}
|
||||||
|
|
||||||
type RemoteWriteFormat int64 //nolint:revive // exported.
|
|
||||||
|
|
||||||
const (
|
const (
|
||||||
Version1 RemoteWriteFormat = iota // 1.0, 0.1, etc.
|
Version1 config.RemoteWriteFormat = iota // 1.0, 0.1, etc.
|
||||||
Version2 // symbols are indices into an array of strings
|
Version2 // symbols are indices into an array of strings
|
||||||
)
|
)
|
||||||
|
|
||||||
|
@ -419,7 +417,7 @@ type QueueManager struct {
|
||||||
watcher *wlog.Watcher
|
watcher *wlog.Watcher
|
||||||
metadataWatcher *MetadataWatcher
|
metadataWatcher *MetadataWatcher
|
||||||
// experimental feature, new remote write proto format
|
// experimental feature, new remote write proto format
|
||||||
rwFormat RemoteWriteFormat
|
rwFormat config.RemoteWriteFormat
|
||||||
|
|
||||||
clientMtx sync.RWMutex
|
clientMtx sync.RWMutex
|
||||||
storeClient WriteClient
|
storeClient WriteClient
|
||||||
|
@ -468,7 +466,7 @@ func NewQueueManager(
|
||||||
sm ReadyScrapeManager,
|
sm ReadyScrapeManager,
|
||||||
enableExemplarRemoteWrite bool,
|
enableExemplarRemoteWrite bool,
|
||||||
enableNativeHistogramRemoteWrite bool,
|
enableNativeHistogramRemoteWrite bool,
|
||||||
rwFormat RemoteWriteFormat,
|
rwFormat config.RemoteWriteFormat,
|
||||||
) *QueueManager {
|
) *QueueManager {
|
||||||
if logger == nil {
|
if logger == nil {
|
||||||
logger = log.NewNopLogger()
|
logger = log.NewNopLogger()
|
||||||
|
@ -1659,16 +1657,15 @@ func populateTimeSeries(batch []timeSeries, pendingData []prompb.TimeSeries, sen
|
||||||
return nPendingSamples, nPendingExemplars, nPendingHistograms
|
return nPendingSamples, nPendingExemplars, nPendingHistograms
|
||||||
}
|
}
|
||||||
|
|
||||||
func (s *shards) sendSamples(ctx context.Context, series []prompb.TimeSeries, sampleCount, exemplarCount, histogramCount int, pBuf *proto.Buffer, buf *[]byte) {
|
func (s *shards) sendSamples(ctx context.Context, samples []prompb.TimeSeries, sampleCount, exemplarCount, histogramCount int, pBuf *proto.Buffer, buf *[]byte) {
|
||||||
begin := time.Now()
|
begin := time.Now()
|
||||||
err := s.sendSamplesWithBackoff(ctx, series, sampleCount, exemplarCount, histogramCount, 0, pBuf, buf)
|
err := s.sendSamplesWithBackoff(ctx, samples, sampleCount, exemplarCount, histogramCount, 0, pBuf, buf)
|
||||||
|
|
||||||
s.updateMetrics(ctx, err, sampleCount, exemplarCount, histogramCount, 0, time.Since(begin))
|
s.updateMetrics(ctx, err, sampleCount, exemplarCount, histogramCount, 0, time.Since(begin))
|
||||||
}
|
}
|
||||||
|
|
||||||
func (s *shards) sendV2Samples(ctx context.Context, samples []writev2.TimeSeries, labels []string, sampleCount, exemplarCount, histogramCount, metadataCount int, pBuf, buf *[]byte) {
|
func (s *shards) sendV2Samples(ctx context.Context, samples []writev2.TimeSeries, labels []string, sampleCount, exemplarCount, histogramCount, metadataCount int, pBuf, buf *[]byte) {
|
||||||
begin := time.Now()
|
begin := time.Now()
|
||||||
err := s.sendV2SamplesWithBackoff(ctx, samples, sampleCount, exemplarCount, histogramCount, metadataCount, labels, pBuf, buf)
|
err := s.sendV2SamplesWithBackoff(ctx, samples, labels, sampleCount, exemplarCount, histogramCount, metadataCount, pBuf, buf)
|
||||||
s.updateMetrics(ctx, err, sampleCount, exemplarCount, histogramCount, metadataCount, time.Since(begin))
|
s.updateMetrics(ctx, err, sampleCount, exemplarCount, histogramCount, metadataCount, time.Since(begin))
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -1786,9 +1783,9 @@ func (s *shards) sendSamplesWithBackoff(ctx context.Context, samples []prompb.Ti
|
||||||
}
|
}
|
||||||
|
|
||||||
// sendV2Samples to the remote storage with backoff for recoverable errors.
|
// sendV2Samples to the remote storage with backoff for recoverable errors.
|
||||||
func (s *shards) sendV2SamplesWithBackoff(ctx context.Context, samples []writev2.TimeSeries, sampleCount, exemplarCount, histogramCount, metadataCount int, labels []string, pBuf, buf *[]byte) error {
|
func (s *shards) sendV2SamplesWithBackoff(ctx context.Context, samples []writev2.TimeSeries, labels []string, sampleCount, exemplarCount, histogramCount, metadataCount int, pBuf, buf *[]byte) error {
|
||||||
// Build the WriteRequest with no metadata.
|
// Build the WriteRequest with no metadata.
|
||||||
req, highest, lowest, err := buildMinimizedWriteRequestStr(s.qm.logger, samples, labels, pBuf, buf, nil)
|
req, highest, lowest, err := buildV2WriteRequest(s.qm.logger, samples, labels, pBuf, buf, nil)
|
||||||
s.qm.buildRequestLimitTimestamp.Store(lowest)
|
s.qm.buildRequestLimitTimestamp.Store(lowest)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
// Failing to build the write request is non-recoverable, since it will
|
// Failing to build the write request is non-recoverable, since it will
|
||||||
|
@ -1807,7 +1804,7 @@ func (s *shards) sendV2SamplesWithBackoff(ctx context.Context, samples []writev2
|
||||||
lowest := s.qm.buildRequestLimitTimestamp.Load()
|
lowest := s.qm.buildRequestLimitTimestamp.Load()
|
||||||
if isSampleOld(currentTime, time.Duration(s.qm.cfg.SampleAgeLimit), lowest) {
|
if isSampleOld(currentTime, time.Duration(s.qm.cfg.SampleAgeLimit), lowest) {
|
||||||
// This will filter out old samples during retries.
|
// This will filter out old samples during retries.
|
||||||
req, _, lowest, err := buildMinimizedWriteRequestStr(
|
req, _, lowest, err := buildV2WriteRequest(
|
||||||
s.qm.logger,
|
s.qm.logger,
|
||||||
samples,
|
samples,
|
||||||
labels,
|
labels,
|
||||||
|
@ -2102,9 +2099,8 @@ func (r *rwSymbolTable) clear() {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
func buildMinimizedWriteRequestStr(logger log.Logger, timeSeries []writev2.TimeSeries, labels []string, pBuf, buf *[]byte, filter func(writev2.TimeSeries) bool) ([]byte, int64, int64, error) {
|
func buildV2WriteRequest(logger log.Logger, samples []writev2.TimeSeries, labels []string, pBuf, buf *[]byte, filter func(writev2.TimeSeries) bool) ([]byte, int64, int64, error) {
|
||||||
highest, lowest, timeSeries,
|
highest, lowest, timeSeries, droppedSamples, droppedExemplars, droppedHistograms := buildV2TimeSeries(samples, filter)
|
||||||
droppedSamples, droppedExemplars, droppedHistograms := buildV2TimeSeries(timeSeries, filter)
|
|
||||||
|
|
||||||
if droppedSamples > 0 || droppedExemplars > 0 || droppedHistograms > 0 {
|
if droppedSamples > 0 || droppedExemplars > 0 || droppedHistograms > 0 {
|
||||||
level.Debug(logger).Log("msg", "dropped data due to their age", "droppedSamples", droppedSamples, "droppedExemplars", droppedExemplars, "droppedHistograms", droppedHistograms)
|
level.Debug(logger).Log("msg", "dropped data due to their age", "droppedSamples", droppedSamples, "droppedExemplars", droppedExemplars, "droppedHistograms", droppedHistograms)
|
||||||
|
|
|
@ -68,7 +68,7 @@ func TestSampleDelivery(t *testing.T) {
|
||||||
exemplars bool
|
exemplars bool
|
||||||
histograms bool
|
histograms bool
|
||||||
floatHistograms bool
|
floatHistograms bool
|
||||||
rwFormat RemoteWriteFormat
|
rwFormat config.RemoteWriteFormat
|
||||||
}{
|
}{
|
||||||
{samples: true, exemplars: false, histograms: false, floatHistograms: false, name: "samples only"},
|
{samples: true, exemplars: false, histograms: false, floatHistograms: false, name: "samples only"},
|
||||||
{samples: true, exemplars: true, histograms: true, floatHistograms: true, name: "samples, exemplars, and histograms"},
|
{samples: true, exemplars: true, histograms: true, floatHistograms: true, name: "samples, exemplars, and histograms"},
|
||||||
|
@ -108,7 +108,7 @@ func TestSampleDelivery(t *testing.T) {
|
||||||
for _, tc := range testcases {
|
for _, tc := range testcases {
|
||||||
t.Run(tc.name, func(t *testing.T) {
|
t.Run(tc.name, func(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, tc.rwFormat, true)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, true)
|
||||||
defer s.Close()
|
defer s.Close()
|
||||||
|
|
||||||
var (
|
var (
|
||||||
|
@ -138,6 +138,8 @@ func TestSampleDelivery(t *testing.T) {
|
||||||
// Apply new config.
|
// Apply new config.
|
||||||
queueConfig.Capacity = len(samples)
|
queueConfig.Capacity = len(samples)
|
||||||
queueConfig.MaxSamplesPerSend = len(samples) / 2
|
queueConfig.MaxSamplesPerSend = len(samples) / 2
|
||||||
|
// For now we only ever have a single rw config in this test.
|
||||||
|
conf.RemoteWriteConfigs[0].ProtocolVersion = tc.rwFormat
|
||||||
require.NoError(t, s.ApplyConfig(conf))
|
require.NoError(t, s.ApplyConfig(conf))
|
||||||
hash, err := toHash(writeConfig)
|
hash, err := toHash(writeConfig)
|
||||||
require.NoError(t, err)
|
require.NoError(t, err)
|
||||||
|
@ -385,7 +387,7 @@ func TestMetadataDelivery(t *testing.T) {
|
||||||
|
|
||||||
func TestWALMetadataDelivery(t *testing.T) {
|
func TestWALMetadataDelivery(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, Version2, true)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, true)
|
||||||
defer s.Close()
|
defer s.Close()
|
||||||
|
|
||||||
cfg := config.DefaultQueueConfig
|
cfg := config.DefaultQueueConfig
|
||||||
|
@ -394,6 +396,7 @@ func TestWALMetadataDelivery(t *testing.T) {
|
||||||
|
|
||||||
writeConfig := baseRemoteWriteConfig("http://test-storage.com")
|
writeConfig := baseRemoteWriteConfig("http://test-storage.com")
|
||||||
writeConfig.QueueConfig = cfg
|
writeConfig.QueueConfig = cfg
|
||||||
|
writeConfig.ProtocolVersion = Version2
|
||||||
|
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.DefaultGlobalConfig,
|
GlobalConfig: config.DefaultGlobalConfig,
|
||||||
|
@ -424,7 +427,7 @@ func TestWALMetadataDelivery(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestSampleDeliveryTimeout(t *testing.T) {
|
func TestSampleDeliveryTimeout(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
// Let's send one less sample than batch size, and wait the timeout duration
|
// Let's send one less sample than batch size, and wait the timeout duration
|
||||||
n := 9
|
n := 9
|
||||||
|
@ -456,7 +459,7 @@ func TestSampleDeliveryTimeout(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestSampleDeliveryOrder(t *testing.T) {
|
func TestSampleDeliveryOrder(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
ts := 10
|
ts := 10
|
||||||
n := config.DefaultQueueConfig.MaxSamplesPerSend * ts
|
n := config.DefaultQueueConfig.MaxSamplesPerSend * ts
|
||||||
|
@ -558,7 +561,7 @@ func TestSeriesReset(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestReshard(t *testing.T) {
|
func TestReshard(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
size := 10 // Make bigger to find more races.
|
size := 10 // Make bigger to find more races.
|
||||||
nSeries := 6
|
nSeries := 6
|
||||||
|
@ -601,7 +604,7 @@ func TestReshard(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestReshardRaceWithStop(t *testing.T) {
|
func TestReshardRaceWithStop(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
c := NewTestWriteClient(rwFormat)
|
c := NewTestWriteClient(rwFormat)
|
||||||
var m *QueueManager
|
var m *QueueManager
|
||||||
|
@ -639,7 +642,7 @@ func TestReshardRaceWithStop(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestReshardPartialBatch(t *testing.T) {
|
func TestReshardPartialBatch(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
samples, series := createTimeseries(1, 10)
|
samples, series := createTimeseries(1, 10)
|
||||||
|
|
||||||
|
@ -685,7 +688,7 @@ func TestReshardPartialBatch(t *testing.T) {
|
||||||
// where a large scrape (> capacity + max samples per send) is appended at the
|
// where a large scrape (> capacity + max samples per send) is appended at the
|
||||||
// same time as a batch times out according to the batch send deadline.
|
// same time as a batch times out according to the batch send deadline.
|
||||||
func TestQueueFilledDeadlock(t *testing.T) {
|
func TestQueueFilledDeadlock(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
samples, series := createTimeseries(50, 1)
|
samples, series := createTimeseries(50, 1)
|
||||||
|
|
||||||
|
@ -727,7 +730,7 @@ func TestQueueFilledDeadlock(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestReleaseNoninternedString(t *testing.T) {
|
func TestReleaseNoninternedString(t *testing.T) {
|
||||||
for _, rwFormat := range []RemoteWriteFormat{Version1, Version2} {
|
for _, rwFormat := range []config.RemoteWriteFormat{Version1, Version2} {
|
||||||
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
t.Run(fmt.Sprint(rwFormat), func(t *testing.T) {
|
||||||
cfg := testDefaultQueueConfig()
|
cfg := testDefaultQueueConfig()
|
||||||
mcfg := config.DefaultMetadataConfig
|
mcfg := config.DefaultMetadataConfig
|
||||||
|
@ -964,10 +967,10 @@ type TestWriteClient struct {
|
||||||
writesReceived int
|
writesReceived int
|
||||||
mtx sync.Mutex
|
mtx sync.Mutex
|
||||||
buf []byte
|
buf []byte
|
||||||
rwFormat RemoteWriteFormat
|
rwFormat config.RemoteWriteFormat
|
||||||
}
|
}
|
||||||
|
|
||||||
func NewTestWriteClient(rwFormat RemoteWriteFormat) *TestWriteClient {
|
func NewTestWriteClient(rwFormat config.RemoteWriteFormat) *TestWriteClient {
|
||||||
return &TestWriteClient{
|
return &TestWriteClient{
|
||||||
receivedSamples: map[string][]prompb.Sample{},
|
receivedSamples: map[string][]prompb.Sample{},
|
||||||
expectedSamples: map[string][]prompb.Sample{},
|
expectedSamples: map[string][]prompb.Sample{},
|
||||||
|
@ -1772,7 +1775,7 @@ func BenchmarkBuildMinimizedWriteRequest(b *testing.B) {
|
||||||
// Warmup buffers
|
// Warmup buffers
|
||||||
for i := 0; i < 10; i++ {
|
for i := 0; i < 10; i++ {
|
||||||
populateV2TimeSeries(&symbolTable, tc.batch, seriesBuff, true, true)
|
populateV2TimeSeries(&symbolTable, tc.batch, seriesBuff, true, true)
|
||||||
buildMinimizedWriteRequestStr(noopLogger, seriesBuff, symbolTable.LabelsStrings(), &pBuf, &buff, nil)
|
buildV2WriteRequest(noopLogger, seriesBuff, symbolTable.LabelsStrings(), &pBuf, &buff, nil)
|
||||||
}
|
}
|
||||||
|
|
||||||
b.Run(fmt.Sprintf("%d-instances", len(tc.batch)), func(b *testing.B) {
|
b.Run(fmt.Sprintf("%d-instances", len(tc.batch)), func(b *testing.B) {
|
||||||
|
@ -1780,7 +1783,7 @@ func BenchmarkBuildMinimizedWriteRequest(b *testing.B) {
|
||||||
for j := 0; j < b.N; j++ {
|
for j := 0; j < b.N; j++ {
|
||||||
populateV2TimeSeries(&symbolTable, tc.batch, seriesBuff, true, true)
|
populateV2TimeSeries(&symbolTable, tc.batch, seriesBuff, true, true)
|
||||||
b.ResetTimer()
|
b.ResetTimer()
|
||||||
req, _, _, err := buildMinimizedWriteRequestStr(noopLogger, seriesBuff, symbolTable.LabelsStrings(), &pBuf, &buff, nil)
|
req, _, _, err := buildV2WriteRequest(noopLogger, seriesBuff, symbolTable.LabelsStrings(), &pBuf, &buff, nil)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
b.Fatal(err)
|
b.Fatal(err)
|
||||||
}
|
}
|
||||||
|
|
|
@ -92,7 +92,7 @@ func TestNoDuplicateReadConfigs(t *testing.T) {
|
||||||
for _, tc := range cases {
|
for _, tc := range cases {
|
||||||
t.Run("", func(t *testing.T) {
|
t.Run("", func(t *testing.T) {
|
||||||
// todo: test with new format type(s)?
|
// todo: test with new format type(s)?
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.DefaultGlobalConfig,
|
GlobalConfig: config.DefaultGlobalConfig,
|
||||||
RemoteReadConfigs: tc.cfgs,
|
RemoteReadConfigs: tc.cfgs,
|
||||||
|
|
|
@ -62,7 +62,7 @@ type Storage struct {
|
||||||
}
|
}
|
||||||
|
|
||||||
// NewStorage returns a remote.Storage.
|
// NewStorage returns a remote.Storage.
|
||||||
func NewStorage(l log.Logger, reg prometheus.Registerer, stCallback startTimeCallback, walDir string, flushDeadline time.Duration, sm ReadyScrapeManager, rwFormat RemoteWriteFormat, metadataInWAL bool) *Storage {
|
func NewStorage(l log.Logger, reg prometheus.Registerer, stCallback startTimeCallback, walDir string, flushDeadline time.Duration, sm ReadyScrapeManager, metadataInWAL bool) *Storage {
|
||||||
if l == nil {
|
if l == nil {
|
||||||
l = log.NewNopLogger()
|
l = log.NewNopLogger()
|
||||||
}
|
}
|
||||||
|
@ -72,7 +72,7 @@ func NewStorage(l log.Logger, reg prometheus.Registerer, stCallback startTimeCal
|
||||||
logger: logger,
|
logger: logger,
|
||||||
localStartTimeCallback: stCallback,
|
localStartTimeCallback: stCallback,
|
||||||
}
|
}
|
||||||
s.rws = NewWriteStorage(s.logger, reg, walDir, flushDeadline, sm, rwFormat, metadataInWAL)
|
s.rws = NewWriteStorage(s.logger, reg, walDir, flushDeadline, sm, metadataInWAL)
|
||||||
return s
|
return s
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -30,7 +30,7 @@ func TestStorageLifecycle(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.DefaultGlobalConfig,
|
GlobalConfig: config.DefaultGlobalConfig,
|
||||||
RemoteWriteConfigs: []*config.RemoteWriteConfig{
|
RemoteWriteConfigs: []*config.RemoteWriteConfig{
|
||||||
|
@ -58,7 +58,7 @@ func TestUpdateRemoteReadConfigs(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
|
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.GlobalConfig{},
|
GlobalConfig: config.GlobalConfig{},
|
||||||
|
@ -80,7 +80,7 @@ func TestFilterExternalLabels(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
|
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.GlobalConfig{
|
GlobalConfig: config.GlobalConfig{
|
||||||
|
@ -106,7 +106,7 @@ func TestIgnoreExternalLabels(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewStorage(nil, nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
|
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.GlobalConfig{
|
GlobalConfig: config.GlobalConfig{
|
||||||
|
@ -158,7 +158,7 @@ func baseRemoteReadConfig(host string) *config.RemoteReadConfig {
|
||||||
// ApplyConfig runs concurrently with Notify
|
// ApplyConfig runs concurrently with Notify
|
||||||
// See https://github.com/prometheus/prometheus/issues/12747
|
// See https://github.com/prometheus/prometheus/issues/12747
|
||||||
func TestWriteStorageApplyConfigsDuringCommit(t *testing.T) {
|
func TestWriteStorageApplyConfigsDuringCommit(t *testing.T) {
|
||||||
s := NewStorage(nil, nil, nil, t.TempDir(), defaultFlushDeadline, nil, Version1, false)
|
s := NewStorage(nil, nil, nil, t.TempDir(), defaultFlushDeadline, nil, false)
|
||||||
|
|
||||||
var wg sync.WaitGroup
|
var wg sync.WaitGroup
|
||||||
wg.Add(2000)
|
wg.Add(2000)
|
||||||
|
|
|
@ -66,7 +66,6 @@ type WriteStorage struct {
|
||||||
externalLabels labels.Labels
|
externalLabels labels.Labels
|
||||||
dir string
|
dir string
|
||||||
queues map[string]*QueueManager
|
queues map[string]*QueueManager
|
||||||
rwFormat RemoteWriteFormat
|
|
||||||
metadataInWAL bool
|
metadataInWAL bool
|
||||||
samplesIn *ewmaRate
|
samplesIn *ewmaRate
|
||||||
flushDeadline time.Duration
|
flushDeadline time.Duration
|
||||||
|
@ -79,13 +78,12 @@ type WriteStorage struct {
|
||||||
}
|
}
|
||||||
|
|
||||||
// NewWriteStorage creates and runs a WriteStorage.
|
// NewWriteStorage creates and runs a WriteStorage.
|
||||||
func NewWriteStorage(logger log.Logger, reg prometheus.Registerer, dir string, flushDeadline time.Duration, sm ReadyScrapeManager, rwFormat RemoteWriteFormat, metadataInWal bool) *WriteStorage {
|
func NewWriteStorage(logger log.Logger, reg prometheus.Registerer, dir string, flushDeadline time.Duration, sm ReadyScrapeManager, metadataInWal bool) *WriteStorage {
|
||||||
if logger == nil {
|
if logger == nil {
|
||||||
logger = log.NewNopLogger()
|
logger = log.NewNopLogger()
|
||||||
}
|
}
|
||||||
rws := &WriteStorage{
|
rws := &WriteStorage{
|
||||||
queues: make(map[string]*QueueManager),
|
queues: make(map[string]*QueueManager),
|
||||||
rwFormat: rwFormat,
|
|
||||||
watcherMetrics: wlog.NewWatcherMetrics(reg),
|
watcherMetrics: wlog.NewWatcherMetrics(reg),
|
||||||
liveReaderMetrics: wlog.NewLiveReaderMetrics(reg),
|
liveReaderMetrics: wlog.NewLiveReaderMetrics(reg),
|
||||||
logger: logger,
|
logger: logger,
|
||||||
|
@ -96,6 +94,7 @@ func NewWriteStorage(logger log.Logger, reg prometheus.Registerer, dir string, f
|
||||||
interner: newPool(),
|
interner: newPool(),
|
||||||
scraper: sm,
|
scraper: sm,
|
||||||
quit: make(chan struct{}),
|
quit: make(chan struct{}),
|
||||||
|
metadataInWAL: metadataInWal,
|
||||||
highestTimestamp: &maxTimestamp{
|
highestTimestamp: &maxTimestamp{
|
||||||
Gauge: prometheus.NewGauge(prometheus.GaugeOpts{
|
Gauge: prometheus.NewGauge(prometheus.GaugeOpts{
|
||||||
Namespace: namespace,
|
Namespace: namespace,
|
||||||
|
@ -149,8 +148,7 @@ func (rws *WriteStorage) ApplyConfig(conf *config.Config) error {
|
||||||
newQueues := make(map[string]*QueueManager)
|
newQueues := make(map[string]*QueueManager)
|
||||||
newHashes := []string{}
|
newHashes := []string{}
|
||||||
for _, rwConf := range conf.RemoteWriteConfigs {
|
for _, rwConf := range conf.RemoteWriteConfigs {
|
||||||
// todo: change the rws.rwFormat to a queue config field
|
if rwConf.ProtocolVersion > Version1 && !rws.metadataInWAL {
|
||||||
if rws.rwFormat > Version1 && rws.metadataInWAL {
|
|
||||||
return errors.New("invalid remote write configuration, if you are using remote write version 2.0 then the feature flag for metadata records in the WAL must be enabled")
|
return errors.New("invalid remote write configuration, if you are using remote write version 2.0 then the feature flag for metadata records in the WAL must be enabled")
|
||||||
}
|
}
|
||||||
hash, err := toHash(rwConf)
|
hash, err := toHash(rwConf)
|
||||||
|
@ -173,7 +171,7 @@ func (rws *WriteStorage) ApplyConfig(conf *config.Config) error {
|
||||||
|
|
||||||
c, err := NewWriteClient(name, &ClientConfig{
|
c, err := NewWriteClient(name, &ClientConfig{
|
||||||
URL: rwConf.URL,
|
URL: rwConf.URL,
|
||||||
RemoteWriteFormat: rws.rwFormat,
|
RemoteWriteFormat: rwConf.ProtocolVersion,
|
||||||
Timeout: rwConf.RemoteTimeout,
|
Timeout: rwConf.RemoteTimeout,
|
||||||
HTTPClientConfig: rwConf.HTTPClientConfig,
|
HTTPClientConfig: rwConf.HTTPClientConfig,
|
||||||
SigV4Config: rwConf.SigV4Config,
|
SigV4Config: rwConf.SigV4Config,
|
||||||
|
@ -216,7 +214,7 @@ func (rws *WriteStorage) ApplyConfig(conf *config.Config) error {
|
||||||
rws.scraper,
|
rws.scraper,
|
||||||
rwConf.SendExemplars,
|
rwConf.SendExemplars,
|
||||||
rwConf.SendNativeHistograms,
|
rwConf.SendNativeHistograms,
|
||||||
rws.rwFormat,
|
rwConf.ProtocolVersion,
|
||||||
)
|
)
|
||||||
// Keep track of which queues are new so we know which to start.
|
// Keep track of which queues are new so we know which to start.
|
||||||
newHashes = append(newHashes, hash)
|
newHashes = append(newHashes, hash)
|
||||||
|
|
|
@ -19,16 +19,16 @@ import (
|
||||||
"fmt"
|
"fmt"
|
||||||
"net/http"
|
"net/http"
|
||||||
|
|
||||||
"github.com/prometheus/prometheus/model/labels"
|
|
||||||
writev2 "github.com/prometheus/prometheus/prompb/write/v2"
|
|
||||||
|
|
||||||
"github.com/go-kit/log"
|
"github.com/go-kit/log"
|
||||||
"github.com/go-kit/log/level"
|
"github.com/go-kit/log/level"
|
||||||
|
|
||||||
"github.com/prometheus/client_golang/prometheus"
|
"github.com/prometheus/client_golang/prometheus"
|
||||||
|
|
||||||
|
"github.com/prometheus/prometheus/config"
|
||||||
"github.com/prometheus/prometheus/model/exemplar"
|
"github.com/prometheus/prometheus/model/exemplar"
|
||||||
|
"github.com/prometheus/prometheus/model/labels"
|
||||||
"github.com/prometheus/prometheus/prompb"
|
"github.com/prometheus/prometheus/prompb"
|
||||||
|
writev2 "github.com/prometheus/prometheus/prompb/write/v2"
|
||||||
"github.com/prometheus/prometheus/storage"
|
"github.com/prometheus/prometheus/storage"
|
||||||
otlptranslator "github.com/prometheus/prometheus/storage/remote/otlptranslator/prometheusremotewrite"
|
otlptranslator "github.com/prometheus/prometheus/storage/remote/otlptranslator/prometheusremotewrite"
|
||||||
)
|
)
|
||||||
|
@ -36,7 +36,7 @@ import (
|
||||||
const (
|
const (
|
||||||
RemoteWriteVersionHeader = "X-Prometheus-Remote-Write-Version"
|
RemoteWriteVersionHeader = "X-Prometheus-Remote-Write-Version"
|
||||||
RemoteWriteVersion1HeaderValue = "0.1.0"
|
RemoteWriteVersion1HeaderValue = "0.1.0"
|
||||||
RemoteWriteVersion11HeaderValue = "1.1" // TODO-RW11: Final value?
|
RemoteWriteVersion2HeaderValue = "2.0"
|
||||||
)
|
)
|
||||||
|
|
||||||
type writeHandler struct {
|
type writeHandler struct {
|
||||||
|
@ -47,13 +47,13 @@ type writeHandler struct {
|
||||||
|
|
||||||
// Experimental feature, new remote write proto format
|
// Experimental feature, new remote write proto format
|
||||||
// The handler will accept the new format, but it can still accept the old one
|
// The handler will accept the new format, but it can still accept the old one
|
||||||
// TODO: this should eventually be via content negotiation
|
// TODO: this should eventually be via content negotiation?
|
||||||
rwFormat RemoteWriteFormat
|
rwFormat config.RemoteWriteFormat
|
||||||
}
|
}
|
||||||
|
|
||||||
// NewWriteHandler creates a http.Handler that accepts remote write requests and
|
// NewWriteHandler creates a http.Handler that accepts remote write requests and
|
||||||
// writes them to the provided appendable.
|
// writes them to the provided appendable.
|
||||||
func NewWriteHandler(logger log.Logger, reg prometheus.Registerer, appendable storage.Appendable, rwFormat RemoteWriteFormat) http.Handler {
|
func NewWriteHandler(logger log.Logger, reg prometheus.Registerer, appendable storage.Appendable, rwFormat config.RemoteWriteFormat) http.Handler {
|
||||||
h := &writeHandler{
|
h := &writeHandler{
|
||||||
logger: logger,
|
logger: logger,
|
||||||
appendable: appendable,
|
appendable: appendable,
|
||||||
|
|
|
@ -85,11 +85,11 @@ func TestRemoteWriteHandler(t *testing.T) {
|
||||||
}
|
}
|
||||||
|
|
||||||
func TestRemoteWriteHandlerMinimizedFormat(t *testing.T) {
|
func TestRemoteWriteHandlerMinimizedFormat(t *testing.T) {
|
||||||
buf, _, _, err := buildMinimizedWriteRequestStr(log.NewNopLogger(), writeRequestMinimizedFixture.Timeseries, writeRequestMinimizedFixture.Symbols, nil, nil, nil)
|
buf, _, _, err := buildV2WriteRequest(log.NewNopLogger(), writeRequestMinimizedFixture.Timeseries, writeRequestMinimizedFixture.Symbols, nil, nil, nil)
|
||||||
require.NoError(t, err)
|
require.NoError(t, err)
|
||||||
|
|
||||||
req, err := http.NewRequest("", "", bytes.NewReader(buf))
|
req, err := http.NewRequest("", "", bytes.NewReader(buf))
|
||||||
req.Header.Set(RemoteWriteVersionHeader, RemoteWriteVersion11HeaderValue)
|
req.Header.Set(RemoteWriteVersionHeader, RemoteWriteVersion2HeaderValue)
|
||||||
require.NoError(t, err)
|
require.NoError(t, err)
|
||||||
|
|
||||||
appendable := &mockAppendable{}
|
appendable := &mockAppendable{}
|
||||||
|
|
|
@ -118,7 +118,7 @@ func TestNoDuplicateWriteConfigs(t *testing.T) {
|
||||||
|
|
||||||
for _, tc := range cases {
|
for _, tc := range cases {
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, nil, dir, time.Millisecond, nil, Version1, false)
|
s := NewWriteStorage(nil, nil, dir, time.Millisecond, nil, false)
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.DefaultGlobalConfig,
|
GlobalConfig: config.DefaultGlobalConfig,
|
||||||
RemoteWriteConfigs: tc.cfgs,
|
RemoteWriteConfigs: tc.cfgs,
|
||||||
|
@ -141,7 +141,7 @@ func TestRestartOnNameChange(t *testing.T) {
|
||||||
require.NoError(t, err)
|
require.NoError(t, err)
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, nil, dir, time.Millisecond, nil, Version1, false)
|
s := NewWriteStorage(nil, nil, dir, time.Millisecond, nil, false)
|
||||||
|
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.DefaultGlobalConfig,
|
GlobalConfig: config.DefaultGlobalConfig,
|
||||||
|
@ -167,7 +167,7 @@ func TestUpdateWithRegisterer(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, prometheus.NewRegistry(), dir, time.Millisecond, nil, Version1, false)
|
s := NewWriteStorage(nil, prometheus.NewRegistry(), dir, time.Millisecond, nil, false)
|
||||||
c1 := &config.RemoteWriteConfig{
|
c1 := &config.RemoteWriteConfig{
|
||||||
Name: "named",
|
Name: "named",
|
||||||
URL: &common_config.URL{
|
URL: &common_config.URL{
|
||||||
|
@ -208,7 +208,7 @@ func TestWriteStorageLifecycle(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewWriteStorage(nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.DefaultGlobalConfig,
|
GlobalConfig: config.DefaultGlobalConfig,
|
||||||
RemoteWriteConfigs: []*config.RemoteWriteConfig{
|
RemoteWriteConfigs: []*config.RemoteWriteConfig{
|
||||||
|
@ -226,7 +226,7 @@ func TestUpdateExternalLabels(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, prometheus.NewRegistry(), dir, time.Second, nil, Version1, false)
|
s := NewWriteStorage(nil, prometheus.NewRegistry(), dir, time.Second, nil, false)
|
||||||
|
|
||||||
externalLabels := labels.FromStrings("external", "true")
|
externalLabels := labels.FromStrings("external", "true")
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
|
@ -256,7 +256,7 @@ func TestWriteStorageApplyConfigsIdempotent(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewWriteStorage(nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
conf := &config.Config{
|
conf := &config.Config{
|
||||||
GlobalConfig: config.GlobalConfig{},
|
GlobalConfig: config.GlobalConfig{},
|
||||||
RemoteWriteConfigs: []*config.RemoteWriteConfig{
|
RemoteWriteConfigs: []*config.RemoteWriteConfig{
|
||||||
|
@ -282,7 +282,7 @@ func TestWriteStorageApplyConfigsPartialUpdate(t *testing.T) {
|
||||||
dir := t.TempDir()
|
dir := t.TempDir()
|
||||||
|
|
||||||
// todo: test with new format type(s)
|
// todo: test with new format type(s)
|
||||||
s := NewWriteStorage(nil, nil, dir, defaultFlushDeadline, nil, Version1, false)
|
s := NewWriteStorage(nil, nil, dir, defaultFlushDeadline, nil, false)
|
||||||
|
|
||||||
c0 := &config.RemoteWriteConfig{
|
c0 := &config.RemoteWriteConfig{
|
||||||
RemoteTimeout: model.Duration(10 * time.Second),
|
RemoteTimeout: model.Duration(10 * time.Second),
|
||||||
|
|
|
@ -88,7 +88,7 @@ func createTestAgentDB(t testing.TB, reg prometheus.Registerer, opts *Options) *
|
||||||
t.Helper()
|
t.Helper()
|
||||||
|
|
||||||
dbDir := t.TempDir()
|
dbDir := t.TempDir()
|
||||||
rs := remote.NewStorage(log.NewNopLogger(), reg, startTime, dbDir, time.Second*30, nil, remote.Version1, false)
|
rs := remote.NewStorage(log.NewNopLogger(), reg, startTime, dbDir, time.Second*30, nil, false)
|
||||||
t.Cleanup(func() {
|
t.Cleanup(func() {
|
||||||
require.NoError(t, rs.Close())
|
require.NoError(t, rs.Close())
|
||||||
})
|
})
|
||||||
|
@ -584,7 +584,7 @@ func TestLockfile(t *testing.T) {
|
||||||
tsdbutil.TestDirLockerUsage(t, func(t *testing.T, data string, createLock bool) (*tsdbutil.DirLocker, testutil.Closer) {
|
tsdbutil.TestDirLockerUsage(t, func(t *testing.T, data string, createLock bool) (*tsdbutil.DirLocker, testutil.Closer) {
|
||||||
logger := log.NewNopLogger()
|
logger := log.NewNopLogger()
|
||||||
reg := prometheus.NewRegistry()
|
reg := prometheus.NewRegistry()
|
||||||
rs := remote.NewStorage(logger, reg, startTime, data, time.Second*30, nil, remote.Version1, false)
|
rs := remote.NewStorage(logger, reg, startTime, data, time.Second*30, nil, false)
|
||||||
t.Cleanup(func() {
|
t.Cleanup(func() {
|
||||||
require.NoError(t, rs.Close())
|
require.NoError(t, rs.Close())
|
||||||
})
|
})
|
||||||
|
@ -604,7 +604,7 @@ func TestLockfile(t *testing.T) {
|
||||||
|
|
||||||
func Test_ExistingWAL_NextRef(t *testing.T) {
|
func Test_ExistingWAL_NextRef(t *testing.T) {
|
||||||
dbDir := t.TempDir()
|
dbDir := t.TempDir()
|
||||||
rs := remote.NewStorage(log.NewNopLogger(), nil, startTime, dbDir, time.Second*30, nil, remote.Version1, false)
|
rs := remote.NewStorage(log.NewNopLogger(), nil, startTime, dbDir, time.Second*30, nil, false)
|
||||||
defer func() {
|
defer func() {
|
||||||
require.NoError(t, rs.Close())
|
require.NoError(t, rs.Close())
|
||||||
}()
|
}()
|
||||||
|
|
|
@ -253,7 +253,7 @@ func NewAPI(
|
||||||
registerer prometheus.Registerer,
|
registerer prometheus.Registerer,
|
||||||
statsRenderer StatsRenderer,
|
statsRenderer StatsRenderer,
|
||||||
rwEnabled bool,
|
rwEnabled bool,
|
||||||
rwFormat remote.RemoteWriteFormat,
|
rwFormat config.RemoteWriteFormat,
|
||||||
otlpEnabled bool,
|
otlpEnabled bool,
|
||||||
) *API {
|
) *API {
|
||||||
a := &API{
|
a := &API{
|
||||||
|
|
|
@ -462,7 +462,7 @@ func TestEndpoints(t *testing.T) {
|
||||||
// TODO: test with other proto format(s)?
|
// TODO: test with other proto format(s)?
|
||||||
remote := remote.NewStorage(promlog.New(&promlogConfig), prometheus.DefaultRegisterer, func() (int64, error) {
|
remote := remote.NewStorage(promlog.New(&promlogConfig), prometheus.DefaultRegisterer, func() (int64, error) {
|
||||||
return 0, nil
|
return 0, nil
|
||||||
}, dbDir, 1*time.Second, nil, remote.Version1, false)
|
}, dbDir, 1*time.Second, nil, false)
|
||||||
|
|
||||||
err = remote.ApplyConfig(&config.Config{
|
err = remote.ApplyConfig(&config.Config{
|
||||||
RemoteReadConfigs: []*config.RemoteReadConfig{
|
RemoteReadConfigs: []*config.RemoteReadConfig{
|
||||||
|
|
|
@ -57,7 +57,6 @@ 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/storage/remote"
|
|
||||||
"github.com/prometheus/prometheus/template"
|
"github.com/prometheus/prometheus/template"
|
||||||
"github.com/prometheus/prometheus/util/httputil"
|
"github.com/prometheus/prometheus/util/httputil"
|
||||||
api_v1 "github.com/prometheus/prometheus/web/api/v1"
|
api_v1 "github.com/prometheus/prometheus/web/api/v1"
|
||||||
|
@ -262,7 +261,8 @@ type Options struct {
|
||||||
EnableOTLPWriteReceiver bool
|
EnableOTLPWriteReceiver bool
|
||||||
IsAgent bool
|
IsAgent bool
|
||||||
AppName string
|
AppName string
|
||||||
RemoteWriteFormat remote.RemoteWriteFormat
|
// TODO(cstyan): should this change to a list of tuples, maybe via the content negotiation PR?
|
||||||
|
RemoteWriteFormat config.RemoteWriteFormat
|
||||||
|
|
||||||
Gatherer prometheus.Gatherer
|
Gatherer prometheus.Gatherer
|
||||||
Registerer prometheus.Registerer
|
Registerer prometheus.Registerer
|
||||||
|
|
Loading…
Reference in a new issue