Skip to content

Commit

Permalink
Use block.MetaFetcher in Store Gateway.
Browse files Browse the repository at this point in the history
Fixes: #1874

* Corrupted disk cache for meta.json is handled gracefully.
* Synchronize was not taking into account deletion by removing meta.json.
* Prepare for future implementation of https://thanos.io/proposals/201901-read-write-operations-bucket.md/
* Better observability for syncronize process.
* More logs for store startup process.

TODO in separate PR:
* More observability for index-cache loading / adding time.

Signed-off-by: Bartlomiej Plotka <bwplotka@gmail.com>
  • Loading branch information
bwplotka committed Jan 6, 2020
1 parent 4d1cc62 commit 63bfaf9
Show file tree
Hide file tree
Showing 6 changed files with 140 additions and 243 deletions.
1 change: 1 addition & 0 deletions CHANGELOG.md
Expand Up @@ -13,6 +13,7 @@ We use *breaking* word for marking changes that are not backward compatible (rel

### Fixed

- []() Store: Improved synchronization of meta JSON files. Store now properly handles corrupted disk cache. Added meta.json sync metrics.
- [#1856](https://github.com/thanos-io/thanos/pull/1856) Receive: close DBReadOnly after flushing to fix a memory leak.
- [#1882](https://github.com/thanos-io/thanos/pull/1882) Receive: upload to object storage as 'receive' rather than 'sidecar'.
- [#1907](https://github.com/thanos-io/thanos/pull/1907) Store: Fixed the duration unit for the metric `thanos_bucket_store_series_gate_duration_seconds`.
Expand Down
22 changes: 17 additions & 5 deletions cmd/thanos/store.go
Expand Up @@ -11,8 +11,10 @@ import (
"github.com/pkg/errors"
"github.com/prometheus/client_golang/prometheus"
"github.com/prometheus/prometheus/pkg/relabel"
"github.com/thanos-io/thanos/pkg/block"
"github.com/thanos-io/thanos/pkg/component"
"github.com/thanos-io/thanos/pkg/extflag"
"github.com/thanos-io/thanos/pkg/extprom"
"github.com/thanos-io/thanos/pkg/model"
"github.com/thanos-io/thanos/pkg/objstore/client"
"github.com/thanos-io/thanos/pkg/prober"
Expand All @@ -26,6 +28,8 @@ import (
yaml "gopkg.in/yaml.v2"
)

const fetcherConcurrency = 32

// registerStore registers a store command.
func registerStore(m map[string]setupFunc, app *kingpin.Application) {
cmd := app.Command(component.Store.String(), "store node giving access to blocks in a bucket provider. Now supported GCS, S3, Azure, Swift and Tencent COS.")
Expand All @@ -47,7 +51,7 @@ func registerStore(m map[string]setupFunc, app *kingpin.Application) {
Default("2GB").Bytes()

maxSampleCount := cmd.Flag("store.grpc.series-sample-limit",
"Maximum amount of samples returned via a single Series call. 0 means no limit. NOTE: for efficiency we take 120 as the number of samples in chunk (it cannot be bigger than that), so the actual number of samples might be lower, even though the maximum could be hit.").
"Maximum amount of samples returned via a single Series call. 0 means no limit. NOTE: For efficiency we take 120 as the number of samples in chunk (it cannot be bigger than that), so the actual number of samples might be lower, even though the maximum could be hit.").
Default("0").Uint()

maxConcurrent := cmd.Flag("store.grpc.series-max-concurrency", "Maximum number of concurrent Series calls.").Default("20").Int()
Expand All @@ -57,7 +61,7 @@ func registerStore(m map[string]setupFunc, app *kingpin.Application) {
syncInterval := cmd.Flag("sync-block-duration", "Repeat interval for syncing the blocks between local and remote view.").
Default("3m").Duration()

blockSyncConcurrency := cmd.Flag("block-sync-concurrency", "Number of goroutines to use when syncing blocks from object storage.").
blockSyncConcurrency := cmd.Flag("block-sync-concurrency", "Number of goroutines to use when constructing index-cache.json blocks from object storage.").
Default("20").Int()

minTime := model.TimeOrDuration(cmd.Flag("min-time", "Start of time range limit to serve. Thanos Store will serve only metrics, which happened later than this value. Option can be a constant time in RFC3339 format or time duration relative to current time, such as -1d or 2h45m. Valid duration units are ms, s, m, h, d, w, y.").
Expand Down Expand Up @@ -128,7 +132,7 @@ func runStore(
indexCacheSizeBytes uint64,
chunkPoolSizeBytes uint64,
maxSampleCount uint64,
maxConcurrent int,
maxConcurrency int,
component component.Component,
verbose bool,
syncInterval time.Duration,
Expand Down Expand Up @@ -202,19 +206,27 @@ func runStore(
return errors.Wrap(err, "create index cache")
}

metaFetcher, err := block.NewMetaFetcher(logger, fetcherConcurrency, bkt, dataDir, extprom.WrapRegistererWithPrefix("thanos_", reg),
block.NewTimePartitionMetaFilter(filterConf.MinTime, filterConf.MaxTime).Filter,
block.NewLabelShardedMetaFilter(relabelConfig).Filter,
)
if err != nil {
return errors.Wrap(err, "meta fetcher")
}

bs, err := store.NewBucketStore(
logger,
reg,
bkt,
metaFetcher,
dataDir,
indexCache,
chunkPoolSizeBytes,
maxSampleCount,
maxConcurrent,
maxConcurrency,
verbose,
blockSyncConcurrency,
filterConf,
relabelConfig,
advertiseCompatibilityLabel,
)
if err != nil {
Expand Down
6 changes: 3 additions & 3 deletions docs/components/store.md
Expand Up @@ -85,7 +85,7 @@ Flags:
for chunks.
--store.grpc.series-sample-limit=0
Maximum amount of samples returned via a single
Series call. 0 means no limit. NOTE: for
Series call. 0 means no limit. NOTE: For
efficiency we take 120 as the number of samples
in chunk (it cannot be bigger than that), so
the actual number of samples might be lower,
Expand All @@ -105,8 +105,8 @@ Flags:
--sync-block-duration=3m Repeat interval for syncing the blocks between
local and remote view.
--block-sync-concurrency=20
Number of goroutines to use when syncing blocks
from object storage.
Number of goroutines to use when constructing
index-cache.json blocks from object storage.
--min-time=0000-01-01T00:00:00Z
Start of time range limit to serve. Thanos
Store will serve only metrics, which happened
Expand Down
126 changes: 31 additions & 95 deletions pkg/store/bucket.go
Expand Up @@ -22,7 +22,6 @@ import (
"github.com/pkg/errors"
"github.com/prometheus/client_golang/prometheus"
"github.com/prometheus/prometheus/pkg/labels"
"github.com/prometheus/prometheus/pkg/relabel"
"github.com/prometheus/prometheus/tsdb/chunkenc"
"github.com/prometheus/prometheus/tsdb/chunks"
"github.com/prometheus/prometheus/tsdb/fileutil"
Expand Down Expand Up @@ -190,7 +189,7 @@ func newBucketStoreMetrics(reg prometheus.Registerer) *bucketStoreMetrics {
return &m
}

// FilterConfig is a configuration, which Store uses for filtering metrics.
// FilterConfig is a configuration, which Store uses for filtering metrics based on time.
type FilterConfig struct {
MinTime, MaxTime model.TimeOrDurationValue
}
Expand All @@ -201,6 +200,7 @@ type BucketStore struct {
logger log.Logger
metrics *bucketStoreMetrics
bucket objstore.BucketReader
fetcher block.MetadataFetcher
dir string
indexCache storecache.IndexCache
chunkPool *pool.BytesPool
Expand All @@ -222,9 +222,7 @@ type BucketStore struct {
samplesLimiter *Limiter
partitioner partitioner

filterConfig *FilterConfig
relabelConfig []*relabel.Config

filterConfig *FilterConfig
advLabelSets []storepb.LabelSet
enableCompatibilityLabel bool
}
Expand All @@ -235,15 +233,15 @@ func NewBucketStore(
logger log.Logger,
reg prometheus.Registerer,
bucket objstore.BucketReader,
fetcher block.MetadataFetcher,
dir string,
indexCache storecache.IndexCache,
maxChunkPoolBytes uint64,
maxSampleCount uint64,
maxConcurrent int,
debugLogging bool,
blockSyncConcurrency int,
filterConf *FilterConfig,
relabelConfig []*relabel.Config,
filterConfig *FilterConfig,
enableCompatibilityLabel bool,
) (*BucketStore, error) {
if logger == nil {
Expand All @@ -265,21 +263,21 @@ func NewBucketStore(
s := &BucketStore{
logger: logger,
bucket: bucket,
fetcher: fetcher,
dir: dir,
indexCache: indexCache,
chunkPool: chunkPool,
blocks: map[ulid.ULID]*bucketBlock{},
blockSets: map[uint64]*bucketBlockSet{},
debugLogging: debugLogging,
blockSyncConcurrency: blockSyncConcurrency,
filterConfig: filterConfig,
queryGate: gate.NewGate(
maxConcurrent,
extprom.WrapRegistererWithPrefix("thanos_bucket_store_series_", reg),
),
samplesLimiter: NewLimiter(maxSampleCount, metrics.queriesDropped),
partitioner: gapBasedPartitioner{maxGapSize: maxGapSize},
filterConfig: filterConf,
relabelConfig: relabelConfig,
enableCompatibilityLabel: enableCompatibilityLabel,
}
s.metrics = metrics
Expand Down Expand Up @@ -310,6 +308,12 @@ func (s *BucketStore) Close() (err error) {
// SyncBlocks synchronizes the stores state with the Bucket bucket.
// It will reuse disk space as persistent cache based on s.dir param.
func (s *BucketStore) SyncBlocks(ctx context.Context) error {
metas, _, metaFetchErr := s.fetcher.Fetch(ctx)
// For partial view allow adding new blocks at least.
if metaFetchErr != nil && metas == nil {
return metaFetchErr
}

var wg sync.WaitGroup
blockc := make(chan *metadata.Meta)

Expand All @@ -318,73 +322,40 @@ func (s *BucketStore) SyncBlocks(ctx context.Context) error {
go func() {
for meta := range blockc {
if err := s.addBlock(ctx, meta); err != nil {
level.Warn(s.logger).Log("msg", "loading block failed", "id", meta.ULID, "err", err)
continue
}
}
wg.Done()
}()
}

allIDs := map[ulid.ULID]struct{}{}

err := s.bucket.Iter(ctx, "", func(name string) error {
// Strip trailing slash indicating a directory.
id, err := ulid.Parse(name[:len(name)-1])
if err != nil {
return nil
}

bdir := path.Join(s.dir, id.String())
meta, err := loadMeta(ctx, s.logger, s.bucket, bdir, id)
if err != nil {
return errors.Wrap(err, "load meta")
}

inRange, err := s.isBlockInMinMaxRange(ctx, meta)
if err != nil {
level.Warn(s.logger).Log("msg", "error parsing block range", "block", id, "err", err)
return os.RemoveAll(bdir)
}

if !inRange {
return os.RemoveAll(bdir)
}

// Check for block labels by relabeling.
// If output is empty, the block will be dropped.
if processedLabels := relabel.Process(labels.FromMap(meta.Thanos.Labels), s.relabelConfig...); processedLabels == nil {
level.Debug(s.logger).Log("msg", "ignoring block (drop in relabeling)", "block", id)
return os.RemoveAll(bdir)
}

allIDs[id] = struct{}{}

for id, meta := range metas {
if b := s.getBlock(id); b != nil {
return nil
continue
}
select {
case <-ctx.Done():
case blockc <- meta:
}
return nil
})
}

close(blockc)
wg.Wait()

if err != nil {
return errors.Wrap(err, "iter")
if metaFetchErr != nil {
return metaFetchErr
}

// Drop all blocks that are no longer present in the bucket.
for id := range s.blocks {
if _, ok := allIDs[id]; ok {
if _, ok := metas[id]; ok {
continue
}
if err := s.removeBlock(id); err != nil {
level.Warn(s.logger).Log("msg", "drop outdated block", "block", id, "err", err)
level.Warn(s.logger).Log("msg", "drop outdated block failed", "block", id, "err", err)
s.metrics.blockDropFailures.Inc()
}
level.Debug(s.logger).Log("msg", "dropped outdated block", "block", id)
s.metrics.blockDrops.Inc()
}

Expand Down Expand Up @@ -436,25 +407,6 @@ func (s *BucketStore) InitialSync(ctx context.Context) error {
return nil
}

func (s *BucketStore) numBlocks() int {
s.mtx.RLock()
defer s.mtx.RUnlock()
return len(s.blocks)
}

func (s *BucketStore) isBlockInMinMaxRange(ctx context.Context, meta *metadata.Meta) (bool, error) {
// We check for blocks in configured minTime, maxTime range.
switch {
case meta.MaxTime <= s.filterConfig.MinTime.PrometheusTimestamp():
return false, nil

case meta.MinTime >= s.filterConfig.MaxTime.PrometheusTimestamp():
return false, nil
}

return true, nil
}

func (s *BucketStore) getBlock(id ulid.ULID) *bucketBlock {
s.mtx.RLock()
defer s.mtx.RUnlock()
Expand All @@ -463,13 +415,22 @@ func (s *BucketStore) getBlock(id ulid.ULID) *bucketBlock {

func (s *BucketStore) addBlock(ctx context.Context, meta *metadata.Meta) (err error) {
dir := filepath.Join(s.dir, meta.ULID.String())
start := time.Now()

if err := os.MkdirAll(dir, os.ModePerm); err != nil {
return errors.Wrap(err, "create dir")
}

level.Debug(s.logger).Log("msg", "loading new block", "id", meta.ULID)
defer func() {
if err != nil {
s.metrics.blockLoadFailures.Inc()
if err2 := os.RemoveAll(dir); err2 != nil {
level.Warn(s.logger).Log("msg", "failed to remove block we cannot load", "err", err2)
}
level.Warn(s.logger).Log("msg", "loading block failed", "elapsed", time.Since(start), "id", meta.ULID, "err", err)
} else {
level.Debug(s.logger).Log("msg", "loaded block", "elapsed", time.Since(start), "id", meta.ULID)
}
}()
s.metrics.blockLoads.Inc()
Expand Down Expand Up @@ -1231,31 +1192,6 @@ func (b *bucketBlock) indexCacheFilename() string {
return path.Join(b.meta.ULID.String(), block.IndexCacheFilename)
}

func loadMeta(ctx context.Context, logger log.Logger, bkt objstore.BucketReader, dir string, id ulid.ULID) (*metadata.Meta, error) {
// If we haven't seen the block before or it is missing the meta.json, download it.
if _, err := os.Stat(path.Join(dir, block.MetaFilename)); os.IsNotExist(err) {
if err := os.MkdirAll(dir, 0777); err != nil {
return nil, errors.Wrap(err, "create dir")
}
src := path.Join(id.String(), block.MetaFilename)

if err := objstore.DownloadFile(ctx, logger, bkt, src, dir); err != nil {
if bkt.IsObjNotFoundErr(errors.Cause(err)) {
level.Debug(logger).Log("msg", "meta file wasn't found. Block not ready or being deleted.", "block", id.String())
}
return nil, errors.Wrap(err, "download meta.json")
}
} else if err != nil {
return nil, err
}
meta, err := metadata.Read(dir)
if err != nil {
return nil, errors.Wrap(err, "read meta.json")
}

return meta, err
}

func (b *bucketBlock) loadIndexCacheFile(ctx context.Context) (err error) {
cachefn := filepath.Join(b.dir, block.IndexCacheFilename)
if err = b.loadIndexCacheFileFromFile(ctx, cachefn); err == nil {
Expand Down

0 comments on commit 63bfaf9

Please sign in to comment.