2021-02-16 11:34:48 +00:00
|
|
|
package shard
|
|
|
|
|
|
|
|
import (
|
|
|
|
"context"
|
2021-02-16 12:17:37 +00:00
|
|
|
"sync"
|
2021-02-16 11:34:48 +00:00
|
|
|
"time"
|
|
|
|
|
2023-04-12 14:35:10 +00:00
|
|
|
"git.frostfs.info/TrueCloudLab/frostfs-node/internal/logs"
|
2023-03-07 13:38:26 +00:00
|
|
|
meta "git.frostfs.info/TrueCloudLab/frostfs-node/pkg/local_object_storage/metabase"
|
|
|
|
"git.frostfs.info/TrueCloudLab/frostfs-node/pkg/local_object_storage/shard/mode"
|
|
|
|
"git.frostfs.info/TrueCloudLab/frostfs-node/pkg/util"
|
|
|
|
"git.frostfs.info/TrueCloudLab/frostfs-node/pkg/util/logger"
|
2023-12-27 15:58:36 +00:00
|
|
|
"git.frostfs.info/TrueCloudLab/frostfs-observability/tracing"
|
2023-07-06 12:36:41 +00:00
|
|
|
objectSDK "git.frostfs.info/TrueCloudLab/frostfs-sdk-go/object"
|
2023-03-07 13:38:26 +00:00
|
|
|
oid "git.frostfs.info/TrueCloudLab/frostfs-sdk-go/object/id"
|
2021-02-16 11:34:48 +00:00
|
|
|
"go.uber.org/zap"
|
2023-03-17 08:06:15 +00:00
|
|
|
"golang.org/x/sync/errgroup"
|
|
|
|
)
|
|
|
|
|
|
|
|
const (
|
|
|
|
minExpiredWorkers = 2
|
|
|
|
minExpiredBatchSize = 1
|
2021-02-16 11:34:48 +00:00
|
|
|
)
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
// TombstoneSource is an interface that checks
|
2023-02-05 15:59:38 +00:00
|
|
|
// tombstone status in the FrostFS network.
|
2022-04-19 18:00:22 +00:00
|
|
|
type TombstoneSource interface {
|
|
|
|
// IsTombstoneAvailable must return boolean value that means
|
2023-02-05 15:59:38 +00:00
|
|
|
// provided tombstone's presence in the FrostFS network at the
|
2022-04-19 18:00:22 +00:00
|
|
|
// time of the passed epoch.
|
2022-05-31 17:00:41 +00:00
|
|
|
IsTombstoneAvailable(ctx context.Context, addr oid.Address, epoch uint64) bool
|
2022-04-19 18:00:22 +00:00
|
|
|
}
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
// Event represents class of external events.
|
|
|
|
type Event interface {
|
|
|
|
typ() eventType
|
|
|
|
}
|
|
|
|
|
|
|
|
type eventType int
|
|
|
|
|
|
|
|
const (
|
|
|
|
_ eventType = iota
|
2021-02-16 17:32:46 +00:00
|
|
|
eventNewEpoch
|
2021-02-16 11:34:48 +00:00
|
|
|
)
|
|
|
|
|
2021-02-16 17:32:46 +00:00
|
|
|
type newEpoch struct {
|
|
|
|
epoch uint64
|
|
|
|
}
|
|
|
|
|
|
|
|
func (e newEpoch) typ() eventType {
|
|
|
|
return eventNewEpoch
|
|
|
|
}
|
|
|
|
|
|
|
|
// EventNewEpoch returns new epoch event.
|
|
|
|
func EventNewEpoch(e uint64) Event {
|
|
|
|
return newEpoch{
|
|
|
|
epoch: e,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
type eventHandler func(context.Context, Event)
|
|
|
|
|
|
|
|
type eventHandlers struct {
|
2021-02-16 12:17:37 +00:00
|
|
|
prevGroup sync.WaitGroup
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
cancelFunc context.CancelFunc
|
|
|
|
|
|
|
|
handlers []eventHandler
|
|
|
|
}
|
|
|
|
|
2023-05-29 14:32:13 +00:00
|
|
|
type gcRunResult struct {
|
|
|
|
success bool
|
|
|
|
deleted uint64
|
|
|
|
failedToDelete uint64
|
|
|
|
}
|
|
|
|
|
|
|
|
const (
|
|
|
|
objectTypeLock = "lock"
|
|
|
|
objectTypeTombstone = "tombstone"
|
|
|
|
objectTypeRegular = "regular"
|
|
|
|
)
|
|
|
|
|
|
|
|
type GCMectrics interface {
|
2024-02-13 06:09:47 +00:00
|
|
|
SetShardID(string)
|
2023-05-29 14:32:13 +00:00
|
|
|
AddRunDuration(d time.Duration, success bool)
|
|
|
|
AddDeletedCount(deleted, failed uint64)
|
|
|
|
AddExpiredObjectCollectionDuration(d time.Duration, success bool, objectType string)
|
|
|
|
AddInhumedObjectCount(count uint64, objectType string)
|
|
|
|
}
|
|
|
|
|
|
|
|
type noopGCMetrics struct{}
|
|
|
|
|
2024-02-13 06:09:47 +00:00
|
|
|
func (m *noopGCMetrics) SetShardID(string) {}
|
2023-05-29 14:32:13 +00:00
|
|
|
func (m *noopGCMetrics) AddRunDuration(time.Duration, bool) {}
|
|
|
|
func (m *noopGCMetrics) AddDeletedCount(uint64, uint64) {}
|
|
|
|
func (m *noopGCMetrics) AddExpiredObjectCollectionDuration(time.Duration, bool, string) {}
|
|
|
|
func (m *noopGCMetrics) AddInhumedObjectCount(uint64, string) {}
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
type gc struct {
|
|
|
|
*gcCfg
|
|
|
|
|
2021-11-11 13:54:36 +00:00
|
|
|
onceStop sync.Once
|
2021-08-04 12:04:37 +00:00
|
|
|
stopChannel chan struct{}
|
2022-10-25 06:08:18 +00:00
|
|
|
wg sync.WaitGroup
|
2021-08-04 12:04:37 +00:00
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
workerPool util.WorkerPool
|
|
|
|
|
2023-05-29 14:32:13 +00:00
|
|
|
remover func(context.Context) gcRunResult
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2024-01-25 08:02:57 +00:00
|
|
|
// eventChan is used only for listening for the new epoch event.
|
|
|
|
// It is ok to keep opened, we are listening for context done when writing in it.
|
2022-09-26 06:30:41 +00:00
|
|
|
eventChan chan Event
|
2021-02-16 11:34:48 +00:00
|
|
|
mEventHandler map[eventType]*eventHandlers
|
|
|
|
}
|
|
|
|
|
|
|
|
type gcCfg struct {
|
|
|
|
removerInterval time.Duration
|
|
|
|
|
|
|
|
log *logger.Logger
|
|
|
|
|
|
|
|
workerPoolInit func(int) util.WorkerPool
|
2023-03-17 08:06:15 +00:00
|
|
|
|
2023-11-21 10:20:36 +00:00
|
|
|
expiredCollectorWorkerCount int
|
|
|
|
expiredCollectorBatchSize int
|
2023-05-29 14:32:13 +00:00
|
|
|
|
|
|
|
metrics GCMectrics
|
2023-08-31 15:17:55 +00:00
|
|
|
|
|
|
|
testHookRemover func(ctx context.Context) gcRunResult
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
|
|
|
|
2022-10-16 12:07:38 +00:00
|
|
|
func defaultGCCfg() gcCfg {
|
|
|
|
return gcCfg{
|
2021-02-16 11:34:48 +00:00
|
|
|
removerInterval: 10 * time.Second,
|
2022-09-28 07:41:01 +00:00
|
|
|
log: &logger.Logger{Logger: zap.L()},
|
2021-02-16 11:34:48 +00:00
|
|
|
workerPoolInit: func(int) util.WorkerPool {
|
|
|
|
return nil
|
|
|
|
},
|
2023-05-29 14:32:13 +00:00
|
|
|
metrics: &noopGCMetrics{},
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2023-03-23 14:59:14 +00:00
|
|
|
func (gc *gc) init(ctx context.Context) {
|
2021-02-16 11:34:48 +00:00
|
|
|
sz := 0
|
|
|
|
|
|
|
|
for _, v := range gc.mEventHandler {
|
|
|
|
sz += len(v.handlers)
|
|
|
|
}
|
|
|
|
|
|
|
|
if sz > 0 {
|
|
|
|
gc.workerPool = gc.workerPoolInit(sz)
|
|
|
|
}
|
|
|
|
|
2022-10-25 06:08:18 +00:00
|
|
|
gc.wg.Add(2)
|
2023-05-29 06:35:08 +00:00
|
|
|
go gc.tickRemover(ctx)
|
2023-03-23 14:59:14 +00:00
|
|
|
go gc.listenEvents(ctx)
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
|
|
|
|
2023-03-23 14:59:14 +00:00
|
|
|
func (gc *gc) listenEvents(ctx context.Context) {
|
2022-10-25 06:08:18 +00:00
|
|
|
defer gc.wg.Done()
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
for {
|
2024-01-25 08:02:57 +00:00
|
|
|
select {
|
|
|
|
case <-gc.stopChannel:
|
|
|
|
gc.log.Warn(logs.ShardStopEventListenerByClosedStopChannel)
|
2021-02-16 11:34:48 +00:00
|
|
|
return
|
2024-01-25 08:02:57 +00:00
|
|
|
case <-ctx.Done():
|
|
|
|
gc.log.Warn(logs.ShardStopEventListenerByContext)
|
|
|
|
return
|
|
|
|
case event, ok := <-gc.eventChan:
|
|
|
|
if !ok {
|
|
|
|
gc.log.Warn(logs.ShardStopEventListenerByClosedEventChannel)
|
|
|
|
return
|
|
|
|
}
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2024-01-25 08:02:57 +00:00
|
|
|
gc.handleEvent(ctx, event)
|
|
|
|
}
|
2023-09-06 15:03:04 +00:00
|
|
|
}
|
|
|
|
}
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2023-09-06 15:03:04 +00:00
|
|
|
func (gc *gc) handleEvent(ctx context.Context, event Event) {
|
|
|
|
v, ok := gc.mEventHandler[event.typ()]
|
|
|
|
if !ok {
|
|
|
|
return
|
|
|
|
}
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2023-09-06 15:03:04 +00:00
|
|
|
v.cancelFunc()
|
|
|
|
v.prevGroup.Wait()
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2023-09-06 15:03:04 +00:00
|
|
|
var runCtx context.Context
|
|
|
|
runCtx, v.cancelFunc = context.WithCancel(ctx)
|
2021-02-16 12:17:37 +00:00
|
|
|
|
2023-09-06 15:03:04 +00:00
|
|
|
v.prevGroup.Add(len(v.handlers))
|
2021-02-19 08:14:51 +00:00
|
|
|
|
2023-09-06 15:03:04 +00:00
|
|
|
for i := range v.handlers {
|
2024-01-25 08:02:57 +00:00
|
|
|
select {
|
|
|
|
case <-ctx.Done():
|
|
|
|
return
|
|
|
|
default:
|
|
|
|
}
|
2023-09-06 15:03:04 +00:00
|
|
|
h := v.handlers[i]
|
2021-02-16 12:17:37 +00:00
|
|
|
|
2023-09-06 15:03:04 +00:00
|
|
|
err := gc.workerPool.Submit(func() {
|
|
|
|
defer v.prevGroup.Done()
|
|
|
|
h(runCtx, event)
|
|
|
|
})
|
|
|
|
if err != nil {
|
|
|
|
gc.log.Warn(logs.ShardCouldNotSubmitGCJobToWorkerPool,
|
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
v.prevGroup.Done()
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2024-01-25 08:02:57 +00:00
|
|
|
func (gc *gc) releaseResources() {
|
|
|
|
if gc.workerPool != nil {
|
|
|
|
gc.workerPool.Release()
|
|
|
|
}
|
|
|
|
|
|
|
|
// Avoid to close gc.eventChan here,
|
|
|
|
// because it is possible that we are close it earlier than stop writing.
|
|
|
|
// It is ok to keep it opened.
|
|
|
|
|
|
|
|
gc.log.Debug(logs.ShardGCIsStopped)
|
|
|
|
}
|
|
|
|
|
2023-05-29 06:35:08 +00:00
|
|
|
func (gc *gc) tickRemover(ctx context.Context) {
|
2022-10-25 06:08:18 +00:00
|
|
|
defer gc.wg.Done()
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
timer := time.NewTimer(gc.removerInterval)
|
|
|
|
defer timer.Stop()
|
|
|
|
|
|
|
|
for {
|
2021-08-04 12:04:37 +00:00
|
|
|
select {
|
2024-01-25 08:02:57 +00:00
|
|
|
case <-ctx.Done():
|
|
|
|
// Context canceled earlier than we start to close shards.
|
|
|
|
// It make sense to stop collecting garbage by context too.
|
|
|
|
gc.releaseResources()
|
|
|
|
return
|
2021-08-04 12:04:37 +00:00
|
|
|
case <-gc.stopChannel:
|
2024-01-25 08:02:57 +00:00
|
|
|
gc.releaseResources()
|
2021-08-04 12:04:37 +00:00
|
|
|
return
|
|
|
|
case <-timer.C:
|
2023-05-29 14:32:13 +00:00
|
|
|
startedAt := time.Now()
|
|
|
|
|
2023-08-31 15:17:55 +00:00
|
|
|
var result gcRunResult
|
|
|
|
if gc.testHookRemover != nil {
|
|
|
|
result = gc.testHookRemover(ctx)
|
|
|
|
} else {
|
|
|
|
result = gc.remover(ctx)
|
|
|
|
}
|
2021-08-04 12:04:37 +00:00
|
|
|
timer.Reset(gc.removerInterval)
|
2023-05-29 14:32:13 +00:00
|
|
|
|
|
|
|
gc.metrics.AddRunDuration(time.Since(startedAt), result.success)
|
|
|
|
gc.metrics.AddDeletedCount(result.deleted, result.failedToDelete)
|
2021-08-04 12:04:37 +00:00
|
|
|
}
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-08-04 12:04:37 +00:00
|
|
|
func (gc *gc) stop() {
|
2021-11-11 13:54:36 +00:00
|
|
|
gc.onceStop.Do(func() {
|
2023-08-31 13:33:37 +00:00
|
|
|
close(gc.stopChannel)
|
2021-11-11 13:54:36 +00:00
|
|
|
})
|
2022-10-25 06:08:18 +00:00
|
|
|
|
2023-04-12 14:35:10 +00:00
|
|
|
gc.log.Info(logs.ShardWaitingForGCWorkersToStop)
|
2022-10-25 06:08:18 +00:00
|
|
|
gc.wg.Wait()
|
2021-08-04 12:04:37 +00:00
|
|
|
}
|
|
|
|
|
2021-04-08 14:19:31 +00:00
|
|
|
// iterates over metabase and deletes objects
|
2021-02-16 11:34:48 +00:00
|
|
|
// with GC-marked graves.
|
2021-12-27 11:04:07 +00:00
|
|
|
// Does nothing if shard is in "read-only" mode.
|
2023-05-29 14:32:13 +00:00
|
|
|
func (s *Shard) removeGarbage(pctx context.Context) (result gcRunResult) {
|
2023-05-29 06:35:08 +00:00
|
|
|
ctx, cancel := context.WithCancel(pctx)
|
2023-05-23 12:02:38 +00:00
|
|
|
defer cancel()
|
|
|
|
|
|
|
|
s.gcCancel.Store(cancel)
|
|
|
|
if s.setModeRequested.Load() {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RLock()
|
|
|
|
defer s.m.RUnlock()
|
|
|
|
|
|
|
|
if s.info.Mode != mode.ReadWrite {
|
2021-12-27 11:04:07 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2023-05-10 14:16:15 +00:00
|
|
|
s.log.Debug(logs.ShardGCRemoveGarbageStarted)
|
|
|
|
defer s.log.Debug(logs.ShardGCRemoveGarbageCompleted)
|
|
|
|
|
2022-05-31 17:00:41 +00:00
|
|
|
buf := make([]oid.Address, 0, s.rmBatchSize)
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2022-05-20 16:48:14 +00:00
|
|
|
var iterPrm meta.GarbageIterationPrm
|
|
|
|
iterPrm.SetHandler(func(g meta.GarbageObject) error {
|
2023-05-23 12:02:38 +00:00
|
|
|
select {
|
|
|
|
case <-ctx.Done():
|
|
|
|
return ctx.Err()
|
|
|
|
default:
|
|
|
|
}
|
|
|
|
|
2021-04-08 14:19:31 +00:00
|
|
|
buf = append(buf, g.Address())
|
2021-02-16 11:34:48 +00:00
|
|
|
|
2021-02-19 08:24:10 +00:00
|
|
|
if len(buf) == s.rmBatchSize {
|
|
|
|
return meta.ErrInterruptIterator
|
|
|
|
}
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
return nil
|
2022-05-20 16:48:14 +00:00
|
|
|
})
|
|
|
|
|
|
|
|
// iterate over metabase's objects with GC mark
|
|
|
|
// (no more than s.rmBatchSize objects)
|
2023-06-06 09:27:19 +00:00
|
|
|
err := s.metaBase.IterateOverGarbage(ctx, iterPrm)
|
2021-02-16 11:34:48 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardIteratorOverMetabaseGraveyardFailed,
|
2021-02-16 11:34:48 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
return
|
|
|
|
} else if len(buf) == 0 {
|
2023-05-29 14:32:13 +00:00
|
|
|
result.success = true
|
2021-02-16 11:34:48 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2022-05-20 18:08:59 +00:00
|
|
|
var deletePrm DeletePrm
|
2022-07-13 12:43:04 +00:00
|
|
|
deletePrm.SetAddresses(buf...)
|
2022-05-20 18:08:59 +00:00
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
// delete accumulated objects
|
2023-10-13 13:28:04 +00:00
|
|
|
res, err := s.delete(ctx, deletePrm, true)
|
2023-05-29 14:32:13 +00:00
|
|
|
|
|
|
|
result.deleted = res.deleted
|
|
|
|
result.failedToDelete = uint64(len(buf)) - res.deleted
|
|
|
|
result.success = true
|
|
|
|
|
2021-02-16 11:34:48 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardCouldNotDeleteTheObjects,
|
2021-02-16 11:34:48 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
2023-05-29 14:32:13 +00:00
|
|
|
result.success = false
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
2023-05-29 14:32:13 +00:00
|
|
|
|
|
|
|
return
|
2021-02-16 11:34:48 +00:00
|
|
|
}
|
2021-02-16 17:34:45 +00:00
|
|
|
|
2023-11-21 10:20:36 +00:00
|
|
|
func (s *Shard) getExpiredObjectsParameters() (workerCount, batchSize int) {
|
2024-02-19 14:03:11 +00:00
|
|
|
workerCount = minExpiredWorkers
|
|
|
|
batchSize = minExpiredBatchSize
|
|
|
|
|
|
|
|
if s.gc.gcCfg.expiredCollectorBatchSize > batchSize {
|
|
|
|
batchSize = s.gc.gcCfg.expiredCollectorBatchSize
|
|
|
|
}
|
|
|
|
|
|
|
|
if s.gc.gcCfg.expiredCollectorWorkerCount > workerCount {
|
|
|
|
workerCount = s.gc.gcCfg.expiredCollectorWorkerCount
|
|
|
|
}
|
2023-03-17 08:06:15 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2021-02-16 17:34:45 +00:00
|
|
|
func (s *Shard) collectExpiredObjects(ctx context.Context, e Event) {
|
2023-05-29 14:32:13 +00:00
|
|
|
var err error
|
|
|
|
startedAt := time.Now()
|
|
|
|
|
|
|
|
defer func() {
|
|
|
|
s.gc.metrics.AddExpiredObjectCollectionDuration(time.Since(startedAt), err == nil, objectTypeRegular)
|
|
|
|
}()
|
|
|
|
|
2023-05-10 14:16:15 +00:00
|
|
|
s.log.Debug(logs.ShardGCCollectingExpiredObjectsStarted, zap.Uint64("epoch", e.(newEpoch).epoch))
|
|
|
|
defer s.log.Debug(logs.ShardGCCollectingExpiredObjectsCompleted, zap.Uint64("epoch", e.(newEpoch).epoch))
|
|
|
|
|
2023-03-17 08:06:15 +00:00
|
|
|
workersCount, batchSize := s.getExpiredObjectsParameters()
|
|
|
|
|
|
|
|
errGroup, egCtx := errgroup.WithContext(ctx)
|
|
|
|
errGroup.SetLimit(workersCount)
|
|
|
|
|
|
|
|
errGroup.Go(func() error {
|
|
|
|
batch := make([]oid.Address, 0, batchSize)
|
2023-05-29 14:32:13 +00:00
|
|
|
expErr := s.getExpiredObjects(egCtx, e.(newEpoch).epoch, func(o *meta.ExpiredObject) {
|
2023-07-06 12:36:41 +00:00
|
|
|
if o.Type() != objectSDK.TypeTombstone && o.Type() != objectSDK.TypeLock {
|
2023-03-17 08:06:15 +00:00
|
|
|
batch = append(batch, o.Address())
|
|
|
|
|
|
|
|
if len(batch) == batchSize {
|
|
|
|
expired := batch
|
|
|
|
errGroup.Go(func() error {
|
|
|
|
s.handleExpiredObjects(egCtx, expired)
|
|
|
|
return egCtx.Err()
|
|
|
|
})
|
|
|
|
batch = make([]oid.Address, 0, batchSize)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
})
|
2023-05-29 14:32:13 +00:00
|
|
|
if expErr != nil {
|
|
|
|
return expErr
|
2021-02-16 17:34:45 +00:00
|
|
|
}
|
2023-03-17 08:06:15 +00:00
|
|
|
|
|
|
|
if len(batch) > 0 {
|
|
|
|
expired := batch
|
|
|
|
errGroup.Go(func() error {
|
|
|
|
s.handleExpiredObjects(egCtx, expired)
|
|
|
|
return egCtx.Err()
|
|
|
|
})
|
|
|
|
}
|
|
|
|
|
|
|
|
return nil
|
|
|
|
})
|
|
|
|
|
2023-05-29 14:32:13 +00:00
|
|
|
if err = errGroup.Wait(); err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardIteratorOverExpiredObjectsFailed, zap.String("error", err.Error()))
|
2023-03-17 08:06:15 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func (s *Shard) handleExpiredObjects(ctx context.Context, expired []oid.Address) {
|
|
|
|
select {
|
|
|
|
case <-ctx.Done():
|
2021-02-16 17:34:45 +00:00
|
|
|
return
|
2023-03-17 08:06:15 +00:00
|
|
|
default:
|
2021-02-16 17:34:45 +00:00
|
|
|
}
|
|
|
|
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RLock()
|
|
|
|
defer s.m.RUnlock()
|
|
|
|
|
|
|
|
if s.info.Mode.NoMetabase() {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2023-06-06 09:27:19 +00:00
|
|
|
expired, err := s.getExpiredWithLinked(ctx, expired)
|
2023-05-15 09:06:38 +00:00
|
|
|
if err != nil {
|
|
|
|
s.log.Warn(logs.ShardGCFailedToGetExpiredWithLinked, zap.Error(err))
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2022-05-20 16:48:14 +00:00
|
|
|
var inhumePrm meta.InhumePrm
|
|
|
|
|
2022-07-12 14:59:37 +00:00
|
|
|
inhumePrm.SetAddresses(expired...)
|
|
|
|
inhumePrm.SetGCMark()
|
2022-05-20 16:48:14 +00:00
|
|
|
|
2021-02-16 17:34:45 +00:00
|
|
|
// inhume the collected objects
|
2023-04-12 14:01:29 +00:00
|
|
|
res, err := s.metaBase.Inhume(ctx, inhumePrm)
|
2021-02-16 17:34:45 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardCouldNotInhumeTheObjects,
|
2021-02-16 17:34:45 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
return
|
|
|
|
}
|
2022-09-09 11:36:34 +00:00
|
|
|
|
2023-12-25 07:15:26 +00:00
|
|
|
s.gc.metrics.AddInhumedObjectCount(res.LogicInhumed(), objectTypeRegular)
|
|
|
|
s.decObjectCounterBy(logical, res.LogicInhumed())
|
2023-12-05 14:00:16 +00:00
|
|
|
s.decObjectCounterBy(user, res.UserInhumed())
|
2023-11-02 10:50:52 +00:00
|
|
|
s.decContainerObjectCounter(res.InhumedByCnrID())
|
2023-02-02 20:28:42 +00:00
|
|
|
|
|
|
|
i := 0
|
|
|
|
for i < res.GetDeletionInfoLength() {
|
|
|
|
delInfo := res.GetDeletionInfoByIndex(i)
|
|
|
|
s.addToContainerSize(delInfo.CID.EncodeToString(), -int64(delInfo.Size))
|
|
|
|
i++
|
|
|
|
}
|
2021-02-16 17:34:45 +00:00
|
|
|
}
|
2021-02-17 12:27:40 +00:00
|
|
|
|
2023-06-06 09:27:19 +00:00
|
|
|
func (s *Shard) getExpiredWithLinked(ctx context.Context, source []oid.Address) ([]oid.Address, error) {
|
2023-05-15 09:06:38 +00:00
|
|
|
result := make([]oid.Address, 0, len(source))
|
2023-06-06 09:27:19 +00:00
|
|
|
parentToChildren, err := s.metaBase.GetChildren(ctx, source)
|
2023-05-15 09:06:38 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
for parent, children := range parentToChildren {
|
|
|
|
result = append(result, parent)
|
|
|
|
result = append(result, children...)
|
|
|
|
}
|
|
|
|
|
|
|
|
return result, nil
|
|
|
|
}
|
|
|
|
|
2021-02-17 12:27:40 +00:00
|
|
|
func (s *Shard) collectExpiredTombstones(ctx context.Context, e Event) {
|
2023-05-29 14:32:13 +00:00
|
|
|
var err error
|
|
|
|
startedAt := time.Now()
|
|
|
|
|
|
|
|
defer func() {
|
|
|
|
s.gc.metrics.AddExpiredObjectCollectionDuration(time.Since(startedAt), err == nil, objectTypeTombstone)
|
|
|
|
}()
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
epoch := e.(newEpoch).epoch
|
|
|
|
log := s.log.With(zap.Uint64("epoch", epoch))
|
|
|
|
|
2023-04-12 14:35:10 +00:00
|
|
|
log.Debug(logs.ShardStartedExpiredTombstonesHandling)
|
2023-05-10 14:16:15 +00:00
|
|
|
defer log.Debug(logs.ShardFinishedExpiredTombstonesHandling)
|
2022-04-19 18:00:22 +00:00
|
|
|
|
|
|
|
const tssDeleteBatch = 50
|
|
|
|
tss := make([]meta.TombstonedObject, 0, tssDeleteBatch)
|
|
|
|
tssExp := make([]meta.TombstonedObject, 0, tssDeleteBatch)
|
|
|
|
|
2022-05-20 16:48:14 +00:00
|
|
|
var iterPrm meta.GraveyardIterationPrm
|
|
|
|
iterPrm.SetHandler(func(deletedObject meta.TombstonedObject) error {
|
2022-04-19 18:00:22 +00:00
|
|
|
tss = append(tss, deletedObject)
|
|
|
|
|
|
|
|
if len(tss) == tssDeleteBatch {
|
|
|
|
return meta.ErrInterruptIterator
|
|
|
|
}
|
|
|
|
|
|
|
|
return nil
|
2022-03-10 17:58:58 +00:00
|
|
|
})
|
2022-04-19 18:00:22 +00:00
|
|
|
|
|
|
|
for {
|
2023-04-12 14:35:10 +00:00
|
|
|
log.Debug(logs.ShardIteratingTombstones)
|
2022-04-19 18:00:22 +00:00
|
|
|
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RLock()
|
|
|
|
|
|
|
|
if s.info.Mode.NoMetabase() {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Debug(logs.ShardShardIsInADegradedModeSkipCollectingExpiredTombstones)
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RUnlock()
|
|
|
|
|
2022-10-26 06:12:09 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2023-06-06 09:27:19 +00:00
|
|
|
err = s.metaBase.IterateOverGraveyard(ctx, iterPrm)
|
2022-01-19 12:12:34 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
log.Error(logs.ShardIteratorOverGraveyardFailed, zap.Error(err))
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RUnlock()
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
return
|
2022-01-19 12:12:34 +00:00
|
|
|
}
|
2022-04-19 18:00:22 +00:00
|
|
|
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RUnlock()
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
tssLen := len(tss)
|
|
|
|
if tssLen == 0 {
|
|
|
|
break
|
|
|
|
}
|
|
|
|
|
|
|
|
for _, ts := range tss {
|
|
|
|
if !s.tsSource.IsTombstoneAvailable(ctx, ts.Tombstone(), epoch) {
|
|
|
|
tssExp = append(tssExp, ts)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2023-04-12 14:35:10 +00:00
|
|
|
log.Debug(logs.ShardHandlingExpiredTombstonesBatch, zap.Int("number", len(tssExp)))
|
2022-04-21 16:18:21 +00:00
|
|
|
s.expiredTombstonesCallback(ctx, tssExp)
|
2022-04-19 18:00:22 +00:00
|
|
|
|
|
|
|
iterPrm.SetOffset(tss[tssLen-1].Address())
|
|
|
|
tss = tss[:0]
|
|
|
|
tssExp = tssExp[:0]
|
2022-01-19 12:12:34 +00:00
|
|
|
}
|
|
|
|
}
|
2021-02-17 12:27:40 +00:00
|
|
|
|
2022-03-10 17:58:58 +00:00
|
|
|
func (s *Shard) collectExpiredLocks(ctx context.Context, e Event) {
|
2023-05-29 14:32:13 +00:00
|
|
|
var err error
|
|
|
|
startedAt := time.Now()
|
|
|
|
|
|
|
|
defer func() {
|
|
|
|
s.gc.metrics.AddExpiredObjectCollectionDuration(time.Since(startedAt), err == nil, objectTypeLock)
|
|
|
|
}()
|
|
|
|
|
2023-05-10 14:16:15 +00:00
|
|
|
s.log.Debug(logs.ShardGCCollectingExpiredLocksStarted, zap.Uint64("epoch", e.(newEpoch).epoch))
|
|
|
|
defer s.log.Debug(logs.ShardGCCollectingExpiredLocksCompleted, zap.Uint64("epoch", e.(newEpoch).epoch))
|
|
|
|
|
2023-03-17 08:06:15 +00:00
|
|
|
workersCount, batchSize := s.getExpiredObjectsParameters()
|
|
|
|
|
|
|
|
errGroup, egCtx := errgroup.WithContext(ctx)
|
|
|
|
errGroup.SetLimit(workersCount)
|
|
|
|
|
|
|
|
errGroup.Go(func() error {
|
|
|
|
batch := make([]oid.Address, 0, batchSize)
|
|
|
|
|
2023-05-29 14:32:13 +00:00
|
|
|
expErr := s.getExpiredObjects(egCtx, e.(newEpoch).epoch, func(o *meta.ExpiredObject) {
|
2023-07-06 12:36:41 +00:00
|
|
|
if o.Type() == objectSDK.TypeLock {
|
2023-03-17 08:06:15 +00:00
|
|
|
batch = append(batch, o.Address())
|
|
|
|
|
|
|
|
if len(batch) == batchSize {
|
|
|
|
expired := batch
|
|
|
|
errGroup.Go(func() error {
|
2023-03-28 08:17:15 +00:00
|
|
|
s.expiredLocksCallback(egCtx, e.(newEpoch).epoch, expired)
|
2023-03-17 08:06:15 +00:00
|
|
|
return egCtx.Err()
|
|
|
|
})
|
|
|
|
batch = make([]oid.Address, 0, batchSize)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
})
|
2023-05-29 14:32:13 +00:00
|
|
|
if expErr != nil {
|
|
|
|
return expErr
|
2022-03-10 17:58:58 +00:00
|
|
|
}
|
|
|
|
|
2023-03-17 08:06:15 +00:00
|
|
|
if len(batch) > 0 {
|
|
|
|
expired := batch
|
|
|
|
errGroup.Go(func() error {
|
2023-03-28 08:17:15 +00:00
|
|
|
s.expiredLocksCallback(egCtx, e.(newEpoch).epoch, expired)
|
2023-03-17 08:06:15 +00:00
|
|
|
return egCtx.Err()
|
|
|
|
})
|
|
|
|
}
|
|
|
|
|
|
|
|
return nil
|
|
|
|
})
|
|
|
|
|
2023-05-29 14:32:13 +00:00
|
|
|
if err = errGroup.Wait(); err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardIteratorOverExpiredLocksFailed, zap.String("error", err.Error()))
|
2023-03-17 08:06:15 +00:00
|
|
|
}
|
2022-03-10 17:58:58 +00:00
|
|
|
}
|
|
|
|
|
2023-03-17 08:06:15 +00:00
|
|
|
func (s *Shard) getExpiredObjects(ctx context.Context, epoch uint64, onExpiredFound func(*meta.ExpiredObject)) error {
|
2022-12-07 17:42:35 +00:00
|
|
|
s.m.RLock()
|
|
|
|
defer s.m.RUnlock()
|
|
|
|
|
|
|
|
if s.info.Mode.NoMetabase() {
|
2023-03-17 08:06:15 +00:00
|
|
|
return ErrDegradedMode
|
2022-10-26 06:12:09 +00:00
|
|
|
}
|
|
|
|
|
2023-06-06 09:27:19 +00:00
|
|
|
err := s.metaBase.IterateExpired(ctx, epoch, func(expiredObject *meta.ExpiredObject) error {
|
2021-02-17 12:27:40 +00:00
|
|
|
select {
|
|
|
|
case <-ctx.Done():
|
|
|
|
return meta.ErrInterruptIterator
|
|
|
|
default:
|
2023-03-17 08:06:15 +00:00
|
|
|
onExpiredFound(expiredObject)
|
2022-01-19 12:12:34 +00:00
|
|
|
return nil
|
2021-02-17 12:27:40 +00:00
|
|
|
}
|
|
|
|
})
|
|
|
|
if err != nil {
|
2023-03-17 08:06:15 +00:00
|
|
|
return err
|
2021-02-17 12:27:40 +00:00
|
|
|
}
|
2023-03-17 08:06:15 +00:00
|
|
|
return ctx.Err()
|
2021-02-17 12:27:40 +00:00
|
|
|
}
|
|
|
|
|
2023-03-28 08:17:15 +00:00
|
|
|
func (s *Shard) selectExpired(ctx context.Context, epoch uint64, addresses []oid.Address) ([]oid.Address, error) {
|
|
|
|
s.m.RLock()
|
|
|
|
defer s.m.RUnlock()
|
|
|
|
|
|
|
|
if s.info.Mode.NoMetabase() {
|
|
|
|
return nil, ErrDegradedMode
|
|
|
|
}
|
|
|
|
|
|
|
|
return s.metaBase.FilterExpired(ctx, epoch, addresses)
|
|
|
|
}
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
// HandleExpiredTombstones marks tombstones themselves as garbage
|
|
|
|
// and clears up corresponding graveyard records.
|
2021-02-17 12:27:40 +00:00
|
|
|
//
|
|
|
|
// Does not modify tss.
|
2023-04-12 14:01:29 +00:00
|
|
|
func (s *Shard) HandleExpiredTombstones(ctx context.Context, tss []meta.TombstonedObject) {
|
2022-10-26 06:12:09 +00:00
|
|
|
if s.GetMode().NoMetabase() {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
// Mark tombstones as garbage.
|
2021-10-14 13:23:21 +00:00
|
|
|
var pInhume meta.InhumePrm
|
|
|
|
|
2022-05-31 17:00:41 +00:00
|
|
|
tsAddrs := make([]oid.Address, 0, len(tss))
|
2022-04-19 18:00:22 +00:00
|
|
|
for _, ts := range tss {
|
|
|
|
tsAddrs = append(tsAddrs, ts.Tombstone())
|
2021-10-14 13:23:21 +00:00
|
|
|
}
|
|
|
|
|
2022-07-12 14:59:37 +00:00
|
|
|
pInhume.SetGCMark()
|
|
|
|
pInhume.SetAddresses(tsAddrs...)
|
2021-10-14 13:23:21 +00:00
|
|
|
|
|
|
|
// inhume tombstones
|
2023-04-12 14:01:29 +00:00
|
|
|
res, err := s.metaBase.Inhume(ctx, pInhume)
|
2021-02-17 12:27:40 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardCouldNotMarkTombstonesAsGarbage,
|
2021-02-17 12:27:40 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
return
|
|
|
|
}
|
2022-04-19 18:00:22 +00:00
|
|
|
|
2023-12-25 07:15:26 +00:00
|
|
|
s.gc.metrics.AddInhumedObjectCount(res.LogicInhumed(), objectTypeTombstone)
|
|
|
|
s.decObjectCounterBy(logical, res.LogicInhumed())
|
2023-12-05 14:00:16 +00:00
|
|
|
s.decObjectCounterBy(user, res.UserInhumed())
|
2023-11-02 10:50:52 +00:00
|
|
|
s.decContainerObjectCounter(res.InhumedByCnrID())
|
2022-09-09 11:36:34 +00:00
|
|
|
|
2023-02-02 20:28:42 +00:00
|
|
|
i := 0
|
|
|
|
for i < res.GetDeletionInfoLength() {
|
|
|
|
delInfo := res.GetDeletionInfoByIndex(i)
|
|
|
|
s.addToContainerSize(delInfo.CID.EncodeToString(), -int64(delInfo.Size))
|
|
|
|
i++
|
|
|
|
}
|
|
|
|
|
2022-04-19 18:00:22 +00:00
|
|
|
// drop just processed expired tombstones
|
|
|
|
// from graveyard
|
2023-06-06 09:27:19 +00:00
|
|
|
err = s.metaBase.DropGraves(ctx, tss)
|
2022-04-19 18:00:22 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardCouldNotDropExpiredGraveRecords, zap.Error(err))
|
2022-04-19 18:00:22 +00:00
|
|
|
}
|
2021-02-17 12:27:40 +00:00
|
|
|
}
|
2022-03-10 17:58:58 +00:00
|
|
|
|
|
|
|
// HandleExpiredLocks unlocks all objects which were locked by lockers.
|
|
|
|
// If successful, marks lockers themselves as garbage.
|
2023-03-28 08:17:15 +00:00
|
|
|
func (s *Shard) HandleExpiredLocks(ctx context.Context, epoch uint64, lockers []oid.Address) {
|
2022-10-26 06:12:09 +00:00
|
|
|
if s.GetMode().NoMetabase() {
|
|
|
|
return
|
|
|
|
}
|
2023-03-28 08:17:15 +00:00
|
|
|
unlocked, err := s.metaBase.FreeLockedBy(lockers)
|
2022-03-10 17:58:58 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardFailureToUnlockObjects,
|
2022-03-10 17:58:58 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
var pInhume meta.InhumePrm
|
2022-07-12 14:59:37 +00:00
|
|
|
pInhume.SetAddresses(lockers...)
|
2023-03-28 08:17:15 +00:00
|
|
|
pInhume.SetForceGCMark()
|
2022-03-10 17:58:58 +00:00
|
|
|
|
2023-04-12 14:01:29 +00:00
|
|
|
res, err := s.metaBase.Inhume(ctx, pInhume)
|
2022-03-10 17:58:58 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardFailureToMarkLockersAsGarbage,
|
2022-03-10 17:58:58 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
return
|
|
|
|
}
|
2022-09-09 11:36:34 +00:00
|
|
|
|
2023-12-25 07:15:26 +00:00
|
|
|
s.gc.metrics.AddInhumedObjectCount(res.LogicInhumed(), objectTypeLock)
|
|
|
|
s.decObjectCounterBy(logical, res.LogicInhumed())
|
2023-12-05 14:00:16 +00:00
|
|
|
s.decObjectCounterBy(user, res.UserInhumed())
|
2023-11-02 10:50:52 +00:00
|
|
|
s.decContainerObjectCounter(res.InhumedByCnrID())
|
2023-02-02 20:28:42 +00:00
|
|
|
|
|
|
|
i := 0
|
|
|
|
for i < res.GetDeletionInfoLength() {
|
|
|
|
delInfo := res.GetDeletionInfoByIndex(i)
|
|
|
|
s.addToContainerSize(delInfo.CID.EncodeToString(), -int64(delInfo.Size))
|
|
|
|
i++
|
|
|
|
}
|
2023-03-28 08:17:15 +00:00
|
|
|
|
|
|
|
s.inhumeUnlockedIfExpired(ctx, epoch, unlocked)
|
|
|
|
}
|
|
|
|
|
|
|
|
func (s *Shard) inhumeUnlockedIfExpired(ctx context.Context, epoch uint64, unlocked []oid.Address) {
|
|
|
|
expiredUnlocked, err := s.selectExpired(ctx, epoch, unlocked)
|
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardFailureToGetExpiredUnlockedObjects, zap.Error(err))
|
2023-03-28 08:17:15 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
if len(expiredUnlocked) == 0 {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
s.handleExpiredObjects(ctx, expiredUnlocked)
|
2022-03-10 17:58:58 +00:00
|
|
|
}
|
2022-05-26 17:38:32 +00:00
|
|
|
|
|
|
|
// HandleDeletedLocks unlocks all objects which were locked by lockers.
|
|
|
|
func (s *Shard) HandleDeletedLocks(lockers []oid.Address) {
|
2022-10-26 06:12:09 +00:00
|
|
|
if s.GetMode().NoMetabase() {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2023-03-28 08:17:15 +00:00
|
|
|
_, err := s.metaBase.FreeLockedBy(lockers)
|
2022-05-26 17:38:32 +00:00
|
|
|
if err != nil {
|
2023-04-12 14:35:10 +00:00
|
|
|
s.log.Warn(logs.ShardFailureToUnlockObjects,
|
2022-05-26 17:38:32 +00:00
|
|
|
zap.String("error", err.Error()),
|
|
|
|
)
|
|
|
|
|
|
|
|
return
|
|
|
|
}
|
|
|
|
}
|
2022-09-26 06:30:41 +00:00
|
|
|
|
|
|
|
// NotificationChannel returns channel for shard events.
|
|
|
|
func (s *Shard) NotificationChannel() chan<- Event {
|
|
|
|
return s.gc.eventChan
|
|
|
|
}
|
2023-12-27 15:58:36 +00:00
|
|
|
|
|
|
|
func (s *Shard) collectExpiredMetrics(ctx context.Context, e Event) {
|
|
|
|
ctx, span := tracing.StartSpanFromContext(ctx, "shard.collectExpiredMetrics")
|
|
|
|
defer span.End()
|
|
|
|
|
|
|
|
epoch := e.(newEpoch).epoch
|
|
|
|
|
|
|
|
s.log.Debug(logs.ShardGCCollectingExpiredMetricsStarted, zap.Uint64("epoch", epoch))
|
|
|
|
defer s.log.Debug(logs.ShardGCCollectingExpiredMetricsCompleted, zap.Uint64("epoch", epoch))
|
|
|
|
|
|
|
|
s.collectExpiredContainerSizeMetrics(ctx, epoch)
|
2023-12-27 15:23:12 +00:00
|
|
|
s.collectExpiredContainerCountMetrics(ctx, epoch)
|
2023-12-27 15:58:36 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
func (s *Shard) collectExpiredContainerSizeMetrics(ctx context.Context, epoch uint64) {
|
|
|
|
ids, err := s.metaBase.ZeroSizeContainers(ctx)
|
|
|
|
if err != nil {
|
|
|
|
s.log.Warn(logs.ShardGCFailedToCollectZeroSizeContainers, zap.Uint64("epoch", epoch), zap.Error(err))
|
|
|
|
return
|
|
|
|
}
|
|
|
|
if len(ids) == 0 {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
s.zeroSizeContainersCallback(ctx, ids)
|
|
|
|
}
|
2023-12-27 15:23:12 +00:00
|
|
|
|
|
|
|
func (s *Shard) collectExpiredContainerCountMetrics(ctx context.Context, epoch uint64) {
|
|
|
|
ids, err := s.metaBase.ZeroCountContainers(ctx)
|
|
|
|
if err != nil {
|
|
|
|
s.log.Warn(logs.ShardGCFailedToCollectZeroCountContainers, zap.Uint64("epoch", epoch), zap.Error(err))
|
|
|
|
return
|
|
|
|
}
|
|
|
|
if len(ids) == 0 {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
s.zeroCountContainersCallback(ctx, ids)
|
|
|
|
}
|