mirror of https://github.com/grafana/loki
Add basic structure of bloom compactor (#10748)
This pull request adds the basic structure for the new bloom compactor component. - Adds new `bloom-compactor` target that runs with with multiple instances joined by a ring - Adds boilerplate functions to index blooms and compact blocks The main goal of this PR is to provide a basic functionality on which future smaller PRs can build upon. Since the code path is completely separate in a new component, it is not used anywhere at the moment.pull/10934/head
parent
abc4ee29c0
commit
0832256d7b
@ -0,0 +1,246 @@ |
||||
/* |
||||
Bloom-compactor |
||||
|
||||
This is a standalone service that is responsible for compacting TSDB indexes into bloomfilters. |
||||
It creates and merges bloomfilters into an aggregated form, called bloom-blocks. |
||||
It maintains a list of references between bloom-blocks and TSDB indexes in files called meta.jsons. |
||||
|
||||
Bloom-compactor regularly runs to check for changes in meta.jsons and runs compaction only upon changes in TSDBs. |
||||
|
||||
bloomCompactor.Compactor |
||||
|
||||
| // Read/Write path
|
||||
bloomshipper.Store** |
||||
| |
||||
bloomshipper.Shipper |
||||
| |
||||
bloomshipper.BloomClient |
||||
| |
||||
ObjectClient |
||||
| |
||||
.....................service boundary |
||||
| |
||||
object storage |
||||
*/ |
||||
package bloomcompactor |
||||
|
||||
import ( |
||||
"context" |
||||
"fmt" |
||||
"os" |
||||
"path/filepath" |
||||
"time" |
||||
|
||||
"github.com/go-kit/log" |
||||
"github.com/grafana/dskit/ring" |
||||
"github.com/grafana/dskit/services" |
||||
"github.com/prometheus/client_golang/prometheus" |
||||
|
||||
"github.com/grafana/loki/pkg/storage" |
||||
v1 "github.com/grafana/loki/pkg/storage/bloom/v1" |
||||
"github.com/grafana/loki/pkg/storage/bloom/v1/filter" |
||||
"github.com/grafana/loki/pkg/storage/config" |
||||
"github.com/grafana/loki/pkg/storage/stores/shipper/bloomshipper" |
||||
) |
||||
|
||||
type Compactor struct { |
||||
services.Service |
||||
|
||||
cfg Config |
||||
logger log.Logger |
||||
bloomCompactorRing ring.ReadRing |
||||
periodConfigs []config.PeriodConfig |
||||
|
||||
// temporary workaround until store has implemented read/write shipper interface
|
||||
bloomShipperClient bloomshipper.Client |
||||
bloomStore bloomshipper.Store |
||||
} |
||||
|
||||
func New(cfg Config, |
||||
readRing ring.ReadRing, |
||||
storageCfg storage.Config, |
||||
periodConfigs []config.PeriodConfig, |
||||
logger log.Logger, |
||||
clientMetrics storage.ClientMetrics, |
||||
_ prometheus.Registerer) (*Compactor, error) { |
||||
c := &Compactor{ |
||||
cfg: cfg, |
||||
logger: logger, |
||||
bloomCompactorRing: readRing, |
||||
periodConfigs: periodConfigs, |
||||
} |
||||
|
||||
client, err := bloomshipper.NewBloomClient(periodConfigs, storageCfg, clientMetrics) |
||||
if err != nil { |
||||
return nil, err |
||||
} |
||||
|
||||
shipper, err := bloomshipper.NewShipper( |
||||
client, |
||||
storageCfg.BloomShipperConfig, |
||||
logger, |
||||
) |
||||
if err != nil { |
||||
return nil, err |
||||
} |
||||
|
||||
store, err := bloomshipper.NewBloomStore(shipper) |
||||
if err != nil { |
||||
return nil, err |
||||
} |
||||
|
||||
// temporary workaround until store has implemented read/write shipper interface
|
||||
c.bloomShipperClient = client |
||||
c.bloomStore = store |
||||
// TODO use a new service with a loop
|
||||
c.Service = services.NewIdleService(c.starting, c.stopping) |
||||
|
||||
return c, nil |
||||
} |
||||
|
||||
func (c *Compactor) starting(_ context.Context) error { |
||||
return nil |
||||
} |
||||
|
||||
func (c *Compactor) stopping(_ error) error { |
||||
return nil |
||||
} |
||||
|
||||
// TODO Get fpRange owned by the compactor instance
|
||||
func NoopGetFingerprintRange() (uint64, uint64) { return 0, 0 } |
||||
|
||||
// TODO List Users from TSDB and add logic to owned user via ring
|
||||
func NoopGetUserID() string { return "" } |
||||
|
||||
// TODO get series from objectClient (TSDB) instead of params
|
||||
func NoopGetSeries() *v1.Series { return nil } |
||||
|
||||
// TODO Then get chunk data from series
|
||||
func NoopGetChunks() []byte { return nil } |
||||
|
||||
// part1: Create a compact method that assumes no block/meta files exists (eg first compaction)
|
||||
// part2: Write logic to check first for existing block/meta files and does above.
|
||||
func (c *Compactor) compactNewChunks(ctx context.Context, dst string) (err error) { |
||||
//part1
|
||||
series := NoopGetSeries() |
||||
data := NoopGetChunks() |
||||
|
||||
bloom := v1.Bloom{Sbf: *filter.NewDefaultScalableBloomFilter(0.01)} |
||||
// create bloom filters from that.
|
||||
bloom.Sbf.Add([]byte(fmt.Sprint(data))) |
||||
|
||||
// block and seriesList
|
||||
seriesList := []v1.SeriesWithBloom{ |
||||
{ |
||||
Series: series, |
||||
Bloom: &bloom, |
||||
}, |
||||
} |
||||
|
||||
writer := v1.NewDirectoryBlockWriter(dst) |
||||
|
||||
builder, err := v1.NewBlockBuilder( |
||||
v1.BlockOptions{ |
||||
SeriesPageSize: 100, |
||||
BloomPageSize: 10 << 10, |
||||
}, writer) |
||||
if err != nil { |
||||
return err |
||||
} |
||||
// BuildFrom closes itself
|
||||
err = builder.BuildFrom(v1.NewSliceIter[v1.SeriesWithBloom](seriesList)) |
||||
if err != nil { |
||||
return err |
||||
} |
||||
|
||||
// TODO Ask Owen, shall we expose a method to expose these paths on BlockWriter?
|
||||
indexPath := filepath.Join(dst, "series") |
||||
bloomPath := filepath.Join(dst, "bloom") |
||||
|
||||
blockRef := bloomshipper.BlockRef{ |
||||
IndexPath: indexPath, |
||||
BlockPath: bloomPath, |
||||
} |
||||
|
||||
blocks := []bloomshipper.Block{ |
||||
{ |
||||
BlockRef: blockRef, |
||||
|
||||
// TODO point to the data to be read
|
||||
Data: nil, |
||||
}, |
||||
} |
||||
|
||||
meta := bloomshipper.Meta{ |
||||
// After successful compaction there should be no tombstones
|
||||
Tombstones: make([]bloomshipper.BlockRef, 0), |
||||
Blocks: []bloomshipper.BlockRef{blockRef}, |
||||
} |
||||
|
||||
err = c.bloomShipperClient.PutMeta(ctx, meta) |
||||
if err != nil { |
||||
return err |
||||
} |
||||
_, err = c.bloomShipperClient.PutBlocks(ctx, blocks) |
||||
if err != nil { |
||||
return err |
||||
} |
||||
// TODO may need to change return value of this func
|
||||
return nil |
||||
} |
||||
|
||||
func (c *Compactor) runCompact(ctx context.Context) error { |
||||
//TODO set MaxLookBackPeriod to Max ingester accepts
|
||||
maxLookBackPeriod := c.cfg.MaxLookBackPeriod |
||||
|
||||
stFp, endFp := NoopGetFingerprintRange() |
||||
tenantID := NoopGetUserID() |
||||
|
||||
end := time.Now().UTC().UnixMilli() |
||||
start := end - maxLookBackPeriod.Milliseconds() |
||||
|
||||
metaSearchParams := bloomshipper.MetaSearchParams{ |
||||
TenantID: tenantID, |
||||
MinFingerprint: stFp, |
||||
MaxFingerprint: endFp, |
||||
StartTimestamp: start, |
||||
EndTimestamp: end, |
||||
} |
||||
|
||||
metas, err := c.bloomShipperClient.GetMetas(ctx, metaSearchParams) |
||||
if err != nil { |
||||
return err |
||||
} |
||||
|
||||
if len(metas) == 0 { |
||||
//run compaction from scratch
|
||||
tempDst := os.TempDir() |
||||
err = c.compactNewChunks(ctx, tempDst) |
||||
if err != nil { |
||||
return err |
||||
} |
||||
} else { |
||||
// part 2
|
||||
// When already compacted metas exists
|
||||
// Deduplicate index paths
|
||||
uniqueIndexPaths := make(map[string]struct{}) |
||||
|
||||
for _, meta := range metas { |
||||
for _, blockRef := range meta.Blocks { |
||||
uniqueIndexPaths[blockRef.IndexPath] = struct{}{} |
||||
} |
||||
} |
||||
|
||||
// TODO complete part 2 - discuss with Owen - add part to compare chunks and blocks.
|
||||
//1. for each period at hand, get TSDB table indexes for given fp range
|
||||
//2. Check blocks for given uniqueIndexPaths and TSDBindexes
|
||||
// if bloomBlock refs are a superset (covers TSDBIndexes plus more outside of range)
|
||||
// create a new meta.json file, tombstone unused index/block paths.
|
||||
|
||||
//else if: there are TSDBindexes that are not covered in bloomBlocks (a subset)
|
||||
//then call compactNewChunks on them and create a new meta.json
|
||||
|
||||
//else: all good, no compaction
|
||||
} |
||||
return nil |
||||
} |
@ -0,0 +1,36 @@ |
||||
package bloomcompactor |
||||
|
||||
import ( |
||||
"flag" |
||||
"time" |
||||
|
||||
"github.com/grafana/loki/pkg/util" |
||||
) |
||||
|
||||
// Config configures the bloom-compactor component.
|
||||
type Config struct { |
||||
// Ring configures the ring store used to save and retrieve the different Bloom-Compactor instances.
|
||||
// In case it isn't explicitly set, it follows the same behavior of the other rings (ex: using the common configuration
|
||||
// section and the ingester configuration by default).
|
||||
RingCfg RingCfg `yaml:"ring,omitempty" doc:"description=Defines the ring to be used by the bloom-compactor servers. In case this isn't configured, this block supports inheriting configuration from the common ring section."` |
||||
// Enabled configures whether bloom-compactors should be used to compact index values into bloomfilters
|
||||
Enabled bool `yaml:"enabled"` |
||||
WorkingDirectory string `yaml:"working_directory"` |
||||
MaxLookBackPeriod time.Duration `yaml:"max_look_back_period"` |
||||
} |
||||
|
||||
// RegisterFlags registers flags for the Bloom-Compactor configuration.
|
||||
func (cfg *Config) RegisterFlags(f *flag.FlagSet) { |
||||
cfg.RingCfg.RegisterFlags("bloom-compactor.", "collectors/", f) |
||||
f.BoolVar(&cfg.Enabled, "bloom-compactor.enabled", false, "Flag to enable or disable the usage of the bloom-compactor component.") |
||||
} |
||||
|
||||
// RingCfg is a wrapper for our internally used ring configuration plus the replication factor.
|
||||
type RingCfg struct { |
||||
// RingConfig configures the Bloom-Compactor ring.
|
||||
util.RingConfig `yaml:",inline"` |
||||
} |
||||
|
||||
func (cfg *RingCfg) RegisterFlags(prefix, storePrefix string, f *flag.FlagSet) { |
||||
cfg.RingConfig.RegisterFlagsWithPrefix(prefix, storePrefix, f) |
||||
} |
@ -0,0 +1,208 @@ |
||||
package bloomcompactor |
||||
|
||||
import ( |
||||
"context" |
||||
"net/http" |
||||
"time" |
||||
|
||||
"github.com/go-kit/log" |
||||
"github.com/go-kit/log/level" |
||||
"github.com/grafana/dskit/kv" |
||||
"github.com/grafana/dskit/ring" |
||||
"github.com/grafana/dskit/services" |
||||
"github.com/pkg/errors" |
||||
"github.com/prometheus/client_golang/prometheus" |
||||
) |
||||
|
||||
const ( |
||||
// ringAutoForgetUnhealthyPeriods is how many consecutive timeout periods an unhealthy instance
|
||||
// in the ring will be automatically removed.
|
||||
ringAutoForgetUnhealthyPeriods = 10 |
||||
|
||||
// ringNameForServer is the name of the ring used by the bloom-compactor server.
|
||||
ringNameForServer = "bloom-compactor" |
||||
// start with a single instance
|
||||
ringNumTokens = 1 |
||||
ringCheckPeriod = 3 * time.Second |
||||
|
||||
// ringKey is the key under which we register different instances of bloom-compactor in the KVStore.
|
||||
ringKey = "bloom-compactor" |
||||
|
||||
replicationFactor = 1 |
||||
) |
||||
|
||||
type RingManager struct { |
||||
services.Service |
||||
|
||||
cfg Config |
||||
logger log.Logger |
||||
|
||||
subservices *services.Manager |
||||
subservicesWatcher *services.FailureWatcher |
||||
|
||||
RingLifecycler *ring.BasicLifecycler |
||||
Ring *ring.Ring |
||||
} |
||||
|
||||
func NewRingManager(cfg Config, logger log.Logger, registerer prometheus.Registerer) (*RingManager, error) { |
||||
rm := &RingManager{ |
||||
cfg: cfg, logger: logger, |
||||
} |
||||
|
||||
// instantiate kv store.
|
||||
ringStore, err := kv.NewClient( |
||||
rm.cfg.RingCfg.KVStore, |
||||
ring.GetCodec(), |
||||
kv.RegistererWithKVName(prometheus.WrapRegistererWithPrefix("loki_", registerer), "bloom-compactor-ring-manager"), |
||||
rm.logger, |
||||
) |
||||
if err != nil { |
||||
return nil, errors.Wrap(err, "bloom-compactor ring manager failed to create KV store client") |
||||
} |
||||
|
||||
lifecyclerCfg, err := rm.cfg.RingCfg.ToLifecyclerConfig(ringNumTokens, rm.logger) |
||||
if err != nil { |
||||
return nil, errors.Wrap(err, "invalid ring lifecycler config") |
||||
} |
||||
|
||||
// Define lifecycler delegates in reverse order (last to be called defined first because they're
|
||||
// chained via "next delegate").
|
||||
delegate := ring.BasicLifecyclerDelegate(rm) |
||||
delegate = ring.NewLeaveOnStoppingDelegate(delegate, rm.logger) |
||||
delegate = ring.NewTokensPersistencyDelegate(rm.cfg.RingCfg.TokensFilePath, ring.JOINING, delegate, rm.logger) |
||||
delegate = ring.NewAutoForgetDelegate(ringAutoForgetUnhealthyPeriods*rm.cfg.RingCfg.HeartbeatTimeout, delegate, rm.logger) |
||||
|
||||
rm.RingLifecycler, err = ring.NewBasicLifecycler(lifecyclerCfg, ringNameForServer, ringKey, ringStore, delegate, rm.logger, registerer) |
||||
if err != nil { |
||||
return nil, errors.Wrap(err, "failed to create bloom-compactor ring manager lifecycler") |
||||
} |
||||
|
||||
// instantiate ring.
|
||||
ringCfg := rm.cfg.RingCfg.ToRingConfig(replicationFactor) |
||||
rm.Ring, err = ring.NewWithStoreClientAndStrategy( |
||||
ringCfg, |
||||
ringNameForServer, |
||||
ringKey, |
||||
ringStore, |
||||
ring.NewIgnoreUnhealthyInstancesReplicationStrategy(), |
||||
prometheus.WrapRegistererWithPrefix("loki_", registerer), |
||||
rm.logger, |
||||
) |
||||
if err != nil { |
||||
return nil, errors.Wrap(err, "bloom-compactor ring manager failed to create ring client") |
||||
} |
||||
|
||||
svcs := []services.Service{rm.RingLifecycler, rm.Ring} |
||||
rm.subservices, err = services.NewManager(svcs...) |
||||
if err != nil { |
||||
return nil, errors.Wrap(err, "new bloom services manager in server mode") |
||||
} |
||||
|
||||
rm.subservicesWatcher = services.NewFailureWatcher() |
||||
rm.subservicesWatcher.WatchManager(rm.subservices) |
||||
rm.Service = services.NewBasicService(rm.starting, rm.running, rm.stopping) |
||||
|
||||
return rm, nil |
||||
|
||||
} |
||||
|
||||
// starting implements the Lifecycler interface and is one of the lifecycle hooks.
|
||||
func (rm *RingManager) starting(ctx context.Context) (err error) { |
||||
// In case this function will return error we want to unregister the instance
|
||||
// from the ring. We do it ensuring dependencies are gracefully stopped if they
|
||||
// were already started.
|
||||
defer func() { |
||||
if err == nil || rm.subservices == nil { |
||||
return |
||||
} |
||||
|
||||
if stopErr := services.StopManagerAndAwaitStopped(context.Background(), rm.subservices); stopErr != nil { |
||||
level.Error(rm.logger).Log("msg", "failed to gracefully stop bloom-compactor ring manager dependencies", "err", stopErr) |
||||
} |
||||
}() |
||||
|
||||
if err := services.StartManagerAndAwaitHealthy(ctx, rm.subservices); err != nil { |
||||
return errors.Wrap(err, "unable to start bloom-compactor ring manager subservices") |
||||
} |
||||
|
||||
// The BasicLifecycler does not automatically move state to ACTIVE such that any additional work that
|
||||
// someone wants to do can be done before becoming ACTIVE. For the bloom-compactor we don't currently
|
||||
// have any additional work so we can become ACTIVE right away.
|
||||
// Wait until the ring client detected this instance in the JOINING
|
||||
// state to make sure that when we'll run the initial sync we already
|
||||
// know the tokens assigned to this instance.
|
||||
level.Info(rm.logger).Log("msg", "waiting until bloom-compactor is JOINING in the ring") |
||||
if err := ring.WaitInstanceState(ctx, rm.Ring, rm.RingLifecycler.GetInstanceID(), ring.JOINING); err != nil { |
||||
return err |
||||
} |
||||
level.Info(rm.logger).Log("msg", "bloom-compactor is JOINING in the ring") |
||||
|
||||
if err = rm.RingLifecycler.ChangeState(ctx, ring.ACTIVE); err != nil { |
||||
return errors.Wrapf(err, "switch instance to %s in the ring", ring.ACTIVE) |
||||
} |
||||
|
||||
// Wait until the ring client detected this instance in the ACTIVE state to
|
||||
// make sure that when we'll run the loop it won't be detected as a ring
|
||||
// topology change.
|
||||
level.Info(rm.logger).Log("msg", "waiting until bloom-compactor is ACTIVE in the ring") |
||||
if err := ring.WaitInstanceState(ctx, rm.Ring, rm.RingLifecycler.GetInstanceID(), ring.ACTIVE); err != nil { |
||||
return err |
||||
} |
||||
level.Info(rm.logger).Log("msg", "bloom-compactor is ACTIVE in the ring") |
||||
|
||||
return nil |
||||
} |
||||
|
||||
// running implements the Lifecycler interface and is one of the lifecycle hooks.
|
||||
func (rm *RingManager) running(ctx context.Context) error { |
||||
t := time.NewTicker(ringCheckPeriod) |
||||
defer t.Stop() |
||||
for { |
||||
select { |
||||
case <-ctx.Done(): |
||||
return nil |
||||
case err := <-rm.subservicesWatcher.Chan(): |
||||
return errors.Wrap(err, "running bloom-compactor ring manager subservice failed") |
||||
case <-t.C: |
||||
continue |
||||
} |
||||
} |
||||
} |
||||
|
||||
// stopping implements the Lifecycler interface and is one of the lifecycle hooks.
|
||||
func (rm *RingManager) stopping(_ error) error { |
||||
level.Debug(rm.logger).Log("msg", "stopping bloom-compactor ring manager") |
||||
return services.StopManagerAndAwaitStopped(context.Background(), rm.subservices) |
||||
} |
||||
|
||||
func (rm *RingManager) ServeHTTP(w http.ResponseWriter, req *http.Request) { |
||||
rm.Ring.ServeHTTP(w, req) |
||||
} |
||||
|
||||
func (rm *RingManager) OnRingInstanceRegister(_ *ring.BasicLifecycler, ringDesc ring.Desc, instanceExists bool, _ string, instanceDesc ring.InstanceDesc) (ring.InstanceState, ring.Tokens) { |
||||
// When we initialize the bloom-compactor instance in the ring we want to start from
|
||||
// a clean situation, so whatever is the state we set it JOINING, while we keep existing
|
||||
// tokens (if any) or the ones loaded from file.
|
||||
var tokens []uint32 |
||||
if instanceExists { |
||||
tokens = instanceDesc.GetTokens() |
||||
} |
||||
|
||||
takenTokens := ringDesc.GetTokens() |
||||
gen := ring.NewRandomTokenGenerator() |
||||
newTokens := gen.GenerateTokens(ringNumTokens-len(tokens), takenTokens) |
||||
|
||||
// Tokens sorting will be enforced by the parent caller.
|
||||
tokens = append(tokens, newTokens...) |
||||
|
||||
return ring.JOINING, tokens |
||||
} |
||||
|
||||
func (rm *RingManager) OnRingInstanceTokens(_ *ring.BasicLifecycler, _ ring.Tokens) { |
||||
} |
||||
|
||||
func (rm *RingManager) OnRingInstanceStopping(_ *ring.BasicLifecycler) { |
||||
} |
||||
|
||||
func (rm *RingManager) OnRingInstanceHeartbeat(_ *ring.BasicLifecycler, _ *ring.Desc, _ *ring.InstanceDesc) { |
||||
} |
Loading…
Reference in new issue