Merge pull request #2419 from prometheus/remove-legacy-remotes

Remove legacy remote storage implementations
This commit is contained in:
Julius Volz 2017-02-14 18:35:39 +01:00 committed by GitHub
commit 980586d183
15 changed files with 79 additions and 279 deletions

View file

@ -34,7 +34,6 @@ import (
"github.com/prometheus/prometheus/storage/local" "github.com/prometheus/prometheus/storage/local"
"github.com/prometheus/prometheus/storage/local/chunk" "github.com/prometheus/prometheus/storage/local/chunk"
"github.com/prometheus/prometheus/storage/local/index" "github.com/prometheus/prometheus/storage/local/index"
"github.com/prometheus/prometheus/storage/remote"
"github.com/prometheus/prometheus/web" "github.com/prometheus/prometheus/web"
) )
@ -52,15 +51,37 @@ var cfg = struct {
notifierTimeout time.Duration notifierTimeout time.Duration
queryEngine promql.EngineOptions queryEngine promql.EngineOptions
web web.Options web web.Options
remote remote.Options
alertmanagerURLs stringset alertmanagerURLs stringset
prometheusURL string prometheusURL string
influxdbURL string
}{ }{
alertmanagerURLs: stringset{}, alertmanagerURLs: stringset{},
} }
// Value type for flags that are now unused, but which are kept around to
// fulfill 1.0 stability guarantees.
type unusedFlag struct {
name string
value string
help string
}
func (f *unusedFlag) Set(v string) error {
f.value = v
log.Warnf("Flag %q is unused, but set to %q! See the flag's help message: %s", f.name, f.value, f.help)
return nil
}
func (f unusedFlag) String() string {
return f.value
}
func registerUnusedFlags(fs *flag.FlagSet, help string, flags []string) {
for _, name := range flags {
fs.Var(&unusedFlag{name: name, help: help}, name, help)
}
}
func init() { func init() {
cfg.fs = flag.NewFlagSet(os.Args[0], flag.ContinueOnError) cfg.fs = flag.NewFlagSet(os.Args[0], flag.ContinueOnError)
cfg.fs.Usage = usage cfg.fs.Usage = usage
@ -190,44 +211,19 @@ func init() {
"Local storage engine. Supported values are: 'persisted' (full local storage with on-disk persistence) and 'none' (no local storage).", "Local storage engine. Supported values are: 'persisted' (full local storage with on-disk persistence) and 'none' (no local storage).",
) )
// Remote storage. // Unused flags for removed remote storage code.
cfg.fs.StringVar( const remoteStorageFlagsHelp = "WARNING: THIS FLAG IS UNUSED! Built-in support for InfluxDB, Graphite, and OpenTSDB has been removed. Use Prometheus's generic remote write feature for building remote storage integrations. See https://prometheus.io/docs/operating/configuration/#<remote_write>"
&cfg.remote.GraphiteAddress, "storage.remote.graphite-address", "", registerUnusedFlags(cfg.fs, remoteStorageFlagsHelp, []string{
"The host:port of the remote Graphite server to send samples to. None, if empty.", "storage.remote.graphite-address",
) "storage.remote.graphite-transport",
cfg.fs.StringVar( "storage.remote.graphite-prefix",
&cfg.remote.GraphiteTransport, "storage.remote.graphite-transport", "tcp", "storage.remote.opentsdb-url",
"Transport protocol to use to communicate with Graphite. 'tcp', if empty.", "storage.remote.influxdb-url",
) "storage.remote.influxdb.retention-policy",
cfg.fs.StringVar( "storage.remote.influxdb.username",
&cfg.remote.GraphitePrefix, "storage.remote.graphite-prefix", "", "storage.remote.influxdb.database",
"The prefix to prepend to all metrics exported to Graphite. None, if empty.", "storage.remote.timeout",
) })
cfg.fs.StringVar(
&cfg.remote.OpentsdbURL, "storage.remote.opentsdb-url", "",
"The URL of the remote OpenTSDB server to send samples to. None, if empty.",
)
cfg.fs.StringVar(
&cfg.influxdbURL, "storage.remote.influxdb-url", "",
"The URL of the remote InfluxDB server to send samples to. None, if empty.",
)
cfg.fs.StringVar(
&cfg.remote.InfluxdbRetentionPolicy, "storage.remote.influxdb.retention-policy", "default",
"The InfluxDB retention policy to use.",
)
cfg.fs.StringVar(
&cfg.remote.InfluxdbUsername, "storage.remote.influxdb.username", "",
"The username to use when sending samples to InfluxDB. The corresponding password must be provided via the INFLUXDB_PW environment variable.",
)
cfg.fs.StringVar(
&cfg.remote.InfluxdbDatabase, "storage.remote.influxdb.database", "prometheus",
"The name of the database to use for storing samples in InfluxDB.",
)
cfg.fs.DurationVar(
&cfg.remote.StorageTimeout, "storage.remote.timeout", 30*time.Second,
"The timeout to use when sending samples to the remote storage.",
)
// Alertmanager. // Alertmanager.
cfg.fs.Var( cfg.fs.Var(
@ -287,17 +283,12 @@ func parse(args []string) error {
// RoutePrefix must always be at least '/'. // RoutePrefix must always be at least '/'.
cfg.web.RoutePrefix = "/" + strings.Trim(cfg.web.RoutePrefix, "/") cfg.web.RoutePrefix = "/" + strings.Trim(cfg.web.RoutePrefix, "/")
if err := parseInfluxdbURL(); err != nil {
return err
}
for u := range cfg.alertmanagerURLs { for u := range cfg.alertmanagerURLs {
if err := validateAlertmanagerURL(u); err != nil { if err := validateAlertmanagerURL(u); err != nil {
return err return err
} }
} }
cfg.remote.InfluxdbPassword = os.Getenv("INFLUXDB_PW")
return nil return nil
} }
@ -332,24 +323,6 @@ func parsePrometheusURL() error {
return nil return nil
} }
func parseInfluxdbURL() error {
if cfg.influxdbURL == "" {
return nil
}
if ok := govalidator.IsURL(cfg.influxdbURL); !ok {
return fmt.Errorf("invalid InfluxDB URL: %s", cfg.influxdbURL)
}
url, err := url.Parse(cfg.influxdbURL)
if err != nil {
return err
}
cfg.remote.InfluxdbURL = url
return nil
}
func validateAlertmanagerURL(u string) error { func validateAlertmanagerURL(u string) error {
if u == "" { if u == "" {
return nil return nil

View file

@ -36,18 +36,6 @@ func TestParse(t *testing.T) {
input: []string{"-web.external-url", "'https://url/prometheus'"}, input: []string{"-web.external-url", "'https://url/prometheus'"},
valid: false, valid: false,
}, },
{
input: []string{"-storage.remote.influxdb-url", ""},
valid: true,
},
{
input: []string{"-storage.remote.influxdb-url", "http://localhost:8086/"},
valid: true,
},
{
input: []string{"-storage.remote.influxdb-url", "'https://some-url/'"},
valid: false,
},
{ {
input: []string{"-alertmanager.url", ""}, input: []string{"-alertmanager.url", ""},
valid: true, valid: true,
@ -69,7 +57,6 @@ func TestParse(t *testing.T) {
for i, test := range tests { for i, test := range tests {
// reset "immutable" config // reset "immutable" config
cfg.prometheusURL = "" cfg.prometheusURL = ""
cfg.influxdbURL = ""
cfg.alertmanagerURLs = stringset{} cfg.alertmanagerURLs = stringset{}
err := parse(test.input) err := parse(test.input)

View file

@ -92,19 +92,9 @@ func Main() int {
return 1 return 1
} }
remoteStorage, err := remote.New(&cfg.remote) remoteStorage := &remote.Storage{}
if err != nil { sampleAppender = append(sampleAppender, remoteStorage)
log.Errorf("Error initializing remote storage: %s", err) reloadables = append(reloadables, remoteStorage)
return 1
}
if remoteStorage != nil {
sampleAppender = append(sampleAppender, remoteStorage)
reloadables = append(reloadables, remoteStorage)
}
reloadableRemoteStorage := remote.NewConfigurable()
sampleAppender = append(sampleAppender, reloadableRemoteStorage)
reloadables = append(reloadables, reloadableRemoteStorage)
var ( var (
notifier = notifier.New(&cfg.notifier) notifier = notifier.New(&cfg.notifier)
@ -188,12 +178,7 @@ func Main() int {
} }
}() }()
if remoteStorage != nil { defer remoteStorage.Stop()
remoteStorage.Start()
defer remoteStorage.Stop()
}
defer reloadableRemoteStorage.Stop()
// The storage has to be fully initialized before registering. // The storage has to be fully initialized before registering.
if instrumentedStorage, ok := localStorage.(prometheus.Collector); ok { if instrumentedStorage, ok := localStorage.(prometheus.Collector); ok {

View file

@ -29,12 +29,13 @@ import (
"github.com/prometheus/client_golang/prometheus" "github.com/prometheus/client_golang/prometheus"
"github.com/prometheus/common/log" "github.com/prometheus/common/log"
"github.com/prometheus/common/model" "github.com/prometheus/common/model"
"github.com/prometheus/prometheus/storage/remote"
"github.com/prometheus/prometheus/storage/remote/graphite"
"github.com/prometheus/prometheus/storage/remote/influxdb"
"github.com/prometheus/prometheus/storage/remote/opentsdb"
influx "github.com/influxdb/influxdb/client" influx "github.com/influxdb/influxdb/client"
"github.com/prometheus/prometheus/documentation/examples/remote_storage/remote_storage_bridge/graphite"
"github.com/prometheus/prometheus/documentation/examples/remote_storage/remote_storage_bridge/influxdb"
"github.com/prometheus/prometheus/documentation/examples/remote_storage/remote_storage_bridge/opentsdb"
"github.com/prometheus/prometheus/storage/remote"
) )
type config struct { type config struct {

View file

@ -1,4 +1,4 @@
// Copyright 2015 The Prometheus Authors // Copyright 2017 The Prometheus Authors
// Licensed under the Apache License, Version 2.0 (the "License"); // Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License. // you may not use this file except in compliance with the License.
// You may obtain a copy of the License at // You may obtain a copy of the License at
@ -14,101 +14,67 @@
package remote package remote
import ( import (
"net/url"
"sync" "sync"
"time"
"github.com/prometheus/client_golang/prometheus"
"github.com/prometheus/common/model" "github.com/prometheus/common/model"
influx "github.com/influxdb/influxdb/client"
"github.com/prometheus/prometheus/config" "github.com/prometheus/prometheus/config"
"github.com/prometheus/prometheus/relabel" "github.com/prometheus/prometheus/relabel"
"github.com/prometheus/prometheus/storage/remote/graphite"
"github.com/prometheus/prometheus/storage/remote/influxdb"
"github.com/prometheus/prometheus/storage/remote/opentsdb"
) )
// Storage collects multiple remote storage queues. // Storage allows queueing samples for remote writes.
type Storage struct { type Storage struct {
queues []*StorageQueueManager
externalLabels model.LabelSet
relabelConfigs []*config.RelabelConfig
mtx sync.RWMutex mtx sync.RWMutex
externalLabels model.LabelSet
conf config.RemoteWriteConfig
queue *StorageQueueManager
} }
// ApplyConfig updates the status state as the new config requires. // ApplyConfig updates the state as the new config requires.
func (s *Storage) ApplyConfig(conf *config.Config) error { func (s *Storage) ApplyConfig(conf *config.Config) error {
s.mtx.Lock() s.mtx.Lock()
defer s.mtx.Unlock() defer s.mtx.Unlock()
s.externalLabels = conf.GlobalConfig.ExternalLabels // TODO: we should only stop & recreate queues which have changes,
s.relabelConfigs = conf.RemoteWriteConfig.WriteRelabelConfigs // as this can be quite disruptive.
return nil var newQueue *StorageQueueManager
}
// New returns a new remote Storage. if conf.RemoteWriteConfig.URL != nil {
func New(o *Options) (*Storage, error) { c, err := NewClient(conf.RemoteWriteConfig)
s := &Storage{} if err != nil {
if o.GraphiteAddress != "" { return err
c := graphite.NewClient(
o.GraphiteAddress, o.GraphiteTransport,
o.StorageTimeout, o.GraphitePrefix)
s.queues = append(s.queues, NewStorageQueueManager(c, nil))
}
if o.OpentsdbURL != "" {
c := opentsdb.NewClient(o.OpentsdbURL, o.StorageTimeout)
s.queues = append(s.queues, NewStorageQueueManager(c, nil))
}
if o.InfluxdbURL != nil {
conf := influx.Config{
URL: *o.InfluxdbURL,
Username: o.InfluxdbUsername,
Password: o.InfluxdbPassword,
Timeout: o.StorageTimeout,
} }
c := influxdb.NewClient(conf, o.InfluxdbDatabase, o.InfluxdbRetentionPolicy) newQueue = NewStorageQueueManager(c, nil)
prometheus.MustRegister(c)
s.queues = append(s.queues, NewStorageQueueManager(c, nil))
} }
if len(s.queues) == 0 {
return nil, nil
}
return s, nil
}
// Options contains configuration parameters for a remote storage. if s.queue != nil {
type Options struct { s.queue.Stop()
StorageTimeout time.Duration
InfluxdbURL *url.URL
InfluxdbRetentionPolicy string
InfluxdbUsername string
InfluxdbPassword string
InfluxdbDatabase string
OpentsdbURL string
GraphiteAddress string
GraphiteTransport string
GraphitePrefix string
}
// Start starts the background processing of the storage queues.
func (s *Storage) Start() {
for _, q := range s.queues {
q.Start()
} }
s.queue = newQueue
s.conf = conf.RemoteWriteConfig
s.externalLabels = conf.GlobalConfig.ExternalLabels
if s.queue != nil {
s.queue.Start()
}
return nil
} }
// Stop the background processing of the storage queues. // Stop the background processing of the storage queues.
func (s *Storage) Stop() { func (s *Storage) Stop() {
for _, q := range s.queues { if s.queue != nil {
q.Stop() s.queue.Stop()
} }
} }
// Append implements storage.SampleAppender. Always returns nil. // Append implements storage.SampleAppender. Always returns nil.
func (s *Storage) Append(smpl *model.Sample) error { func (s *Storage) Append(smpl *model.Sample) error {
s.mtx.RLock() s.mtx.RLock()
defer s.mtx.RUnlock()
if s.queue == nil {
return nil
}
var snew model.Sample var snew model.Sample
snew = *smpl snew = *smpl
@ -120,16 +86,12 @@ func (s *Storage) Append(smpl *model.Sample) error {
} }
} }
snew.Metric = model.Metric( snew.Metric = model.Metric(
relabel.Process(model.LabelSet(snew.Metric), s.relabelConfigs...)) relabel.Process(model.LabelSet(snew.Metric), s.conf.WriteRelabelConfigs...))
s.mtx.RUnlock()
if snew.Metric == nil { if snew.Metric == nil {
return nil return nil
} }
s.queue.Append(&snew)
for _, q := range s.queues {
q.Append(&snew)
}
return nil return nil
} }

View file

@ -1,108 +0,0 @@
// Copyright 2016 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 remote
import (
"sync"
"github.com/prometheus/common/model"
"github.com/prometheus/prometheus/config"
"github.com/prometheus/prometheus/relabel"
)
// Storage collects multiple remote storage queues.
type ReloadableStorage struct {
mtx sync.RWMutex
externalLabels model.LabelSet
conf config.RemoteWriteConfig
queue *StorageQueueManager
}
// New returns a new remote Storage.
func NewConfigurable() *ReloadableStorage {
return &ReloadableStorage{}
}
// ApplyConfig updates the state as the new config requires.
func (s *ReloadableStorage) ApplyConfig(conf *config.Config) error {
s.mtx.Lock()
defer s.mtx.Unlock()
// TODO: we should only stop & recreate queues which have changes,
// as this can be quite disruptive.
var newQueue *StorageQueueManager
if conf.RemoteWriteConfig.URL != nil {
c, err := NewClient(conf.RemoteWriteConfig)
if err != nil {
return err
}
newQueue = NewStorageQueueManager(c, nil)
}
if s.queue != nil {
s.queue.Stop()
}
s.queue = newQueue
s.conf = conf.RemoteWriteConfig
s.externalLabels = conf.GlobalConfig.ExternalLabels
if s.queue != nil {
s.queue.Start()
}
return nil
}
// Stop the background processing of the storage queues.
func (s *ReloadableStorage) Stop() {
if s.queue != nil {
s.queue.Stop()
}
}
// Append implements storage.SampleAppender. Always returns nil.
func (s *ReloadableStorage) Append(smpl *model.Sample) error {
s.mtx.RLock()
defer s.mtx.RUnlock()
if s.queue == nil {
return nil
}
var snew model.Sample
snew = *smpl
snew.Metric = smpl.Metric.Clone()
for ln, lv := range s.externalLabels {
if _, ok := smpl.Metric[ln]; !ok {
snew.Metric[ln] = lv
}
}
snew.Metric = model.Metric(
relabel.Process(model.LabelSet(snew.Metric), s.conf.WriteRelabelConfigs...))
if snew.Metric == nil {
return nil
}
s.queue.Append(&snew)
return nil
}
// NeedsThrottling implements storage.SampleAppender. It will always return
// false as a remote storage drops samples on the floor if backlogging instead
// of asking for throttling.
func (s *ReloadableStorage) NeedsThrottling() bool {
return false
}