forked from TrueCloudLab/frostfs-node
Initial commit
Initial public review release v0.10.0
This commit is contained in:
commit
dadfd90dcd
276 changed files with 46331 additions and 0 deletions
197
lib/replication/common.go
Normal file
197
lib/replication/common.go
Normal file
|
@ -0,0 +1,197 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
|
||||
"github.com/multiformats/go-multiaddr"
|
||||
"github.com/nspcc-dev/neofs-api-go/object"
|
||||
"github.com/nspcc-dev/neofs-api-go/refs"
|
||||
"github.com/pkg/errors"
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// CID is a type alias of
|
||||
// CID from refs package of neofs-api-go.
|
||||
CID = refs.CID
|
||||
|
||||
// Object is a type alias of
|
||||
// Object from object package of neofs-api-go.
|
||||
Object = object.Object
|
||||
|
||||
// OwnerID is a type alias of
|
||||
// OwnerID from object package of neofs-api-go.
|
||||
OwnerID = object.OwnerID
|
||||
|
||||
// Address is a type alias of
|
||||
// Address from refs package of neofs-api-go.
|
||||
Address = refs.Address
|
||||
|
||||
// ObjectVerificationParams groups the parameters of stored object verification.
|
||||
ObjectVerificationParams struct {
|
||||
Address
|
||||
Node multiaddr.Multiaddr
|
||||
Handler func(valid bool, obj *Object)
|
||||
LocalInvalid bool
|
||||
}
|
||||
|
||||
// ObjectVerifier is an interface of stored object verifier.
|
||||
ObjectVerifier interface {
|
||||
Verify(ctx context.Context, params *ObjectVerificationParams) bool
|
||||
}
|
||||
|
||||
// ObjectSource is an interface of the object storage with read access.
|
||||
ObjectSource interface {
|
||||
Get(ctx context.Context, addr Address) (*Object, error)
|
||||
}
|
||||
|
||||
// ObjectStoreParams groups the parameters for object storing.
|
||||
ObjectStoreParams struct {
|
||||
*Object
|
||||
Nodes []ObjectLocation
|
||||
Handler func(ObjectLocation, bool)
|
||||
}
|
||||
|
||||
// ObjectReceptacle is an interface of object storage with write access.
|
||||
ObjectReceptacle interface {
|
||||
Put(ctx context.Context, params ObjectStoreParams) error
|
||||
}
|
||||
|
||||
// ObjectCleaner Entity for removing object by address from somewhere
|
||||
ObjectCleaner interface {
|
||||
Del(Address) error
|
||||
}
|
||||
|
||||
// ContainerActualityChecker is an interface of entity
|
||||
// for checking local node presence in container
|
||||
// Return true if no errors && local node is in container
|
||||
ContainerActualityChecker interface {
|
||||
Actual(ctx context.Context, cid CID) bool
|
||||
}
|
||||
|
||||
// ObjectPool is a queue of objects selected for data audit.
|
||||
// It is updated once in epoch.
|
||||
ObjectPool interface {
|
||||
Update([]Address)
|
||||
Pop() (Address, error)
|
||||
Undone() int
|
||||
}
|
||||
|
||||
// Scheduler returns slice of addresses for data audit.
|
||||
// These addresses put into ObjectPool.
|
||||
Scheduler interface {
|
||||
SelectForReplication(limit int) ([]Address, error)
|
||||
}
|
||||
|
||||
// ReservationRatioReceiver is an interface of entity
|
||||
// for getting reservation ratio value of object by address.
|
||||
ReservationRatioReceiver interface {
|
||||
ReservationRatio(ctx context.Context, objAddr Address) (int, error)
|
||||
}
|
||||
|
||||
// RemoteStorageSelector is an interface of entity
|
||||
// for getting remote nodes from placement for object by address
|
||||
// Result doesn't contain nodes from exclude list
|
||||
RemoteStorageSelector interface {
|
||||
SelectRemoteStorages(ctx context.Context, addr Address, excl ...multiaddr.Multiaddr) ([]ObjectLocation, error)
|
||||
}
|
||||
|
||||
// MultiSolver is an interface that encapsulates other different utilities.
|
||||
MultiSolver interface {
|
||||
AddressStore
|
||||
RemoteStorageSelector
|
||||
ReservationRatioReceiver
|
||||
ContainerActualityChecker
|
||||
EpochReceiver
|
||||
WeightComparator
|
||||
}
|
||||
|
||||
// ObjectLocator is an itnerface of entity
|
||||
// for building list current object remote nodes by address
|
||||
ObjectLocator interface {
|
||||
LocateObject(ctx context.Context, objAddr Address) ([]multiaddr.Multiaddr, error)
|
||||
}
|
||||
|
||||
// WeightComparator is an itnerface of entity
|
||||
// for comparing weight by address of local node with passed node
|
||||
// returns -1 if local node is weightier or on error
|
||||
// returns 0 if weights are equal
|
||||
// returns 1 if passed node is weightier
|
||||
WeightComparator interface {
|
||||
CompareWeight(ctx context.Context, addr Address, node multiaddr.Multiaddr) int
|
||||
}
|
||||
|
||||
// EpochReceiver is an interface of entity for getting current epoch number.
|
||||
EpochReceiver interface {
|
||||
Epoch() uint64
|
||||
}
|
||||
|
||||
// ObjectLocation groups the information about object current remote location.
|
||||
ObjectLocation struct {
|
||||
Node multiaddr.Multiaddr
|
||||
WeightGreater bool // true if Node field value has less index in placement vector than localhost
|
||||
}
|
||||
|
||||
// ObjectLocationRecord groups the information about all current locations.
|
||||
ObjectLocationRecord struct {
|
||||
Address
|
||||
ReservationRatio int
|
||||
Locations []ObjectLocation
|
||||
}
|
||||
|
||||
// ReplicateTask groups the information about object replication task.
|
||||
// Task solver should not process nodes from exclude list,
|
||||
// Task solver should perform up to Shortage replications.
|
||||
ReplicateTask struct {
|
||||
Address
|
||||
Shortage int
|
||||
ExcludeNodes []multiaddr.Multiaddr
|
||||
}
|
||||
|
||||
// ReplicateResult groups the information about object replication task result.
|
||||
ReplicateResult struct {
|
||||
*ReplicateTask
|
||||
NewStorages []multiaddr.Multiaddr
|
||||
}
|
||||
|
||||
// PresenceChecker is an interface of object storage with presence check access.
|
||||
PresenceChecker interface {
|
||||
Has(address Address) (bool, error)
|
||||
}
|
||||
|
||||
// AddressStore is an interface of local peer's network address storage.
|
||||
AddressStore interface {
|
||||
SelfAddr() (multiaddr.Multiaddr, error)
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
writeResultTimeout = "write result timeout"
|
||||
|
||||
taskChanClosed = " process finish finish: task channel closed"
|
||||
ctxDoneMsg = " process finish: context done"
|
||||
|
||||
objectPoolPart = "object pool"
|
||||
loggerPart = "logger"
|
||||
objectVerifierPart = "object verifier"
|
||||
objectReceptaclePart = "object receptacle"
|
||||
remoteStorageSelectorPart = "remote storage elector"
|
||||
objectSourcePart = "object source"
|
||||
reservationRatioReceiverPart = "reservation ratio receiver"
|
||||
objectLocatorPart = "object locator"
|
||||
epochReceiverPart = "epoch receiver"
|
||||
presenceCheckerPart = "object presence checker"
|
||||
weightComparatorPart = "weight comparator"
|
||||
addrStorePart = "address store"
|
||||
)
|
||||
|
||||
func instanceError(entity, part string) error {
|
||||
return errors.Errorf("could not instantiate %s: empty %s", entity, part)
|
||||
}
|
||||
|
||||
func addressFields(addr Address) []zap.Field {
|
||||
return []zap.Field{
|
||||
zap.Stringer("oid", addr.ObjectID),
|
||||
zap.Stringer("cid", addr.CID),
|
||||
}
|
||||
}
|
27
lib/replication/garbage.go
Normal file
27
lib/replication/garbage.go
Normal file
|
@ -0,0 +1,27 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"sync"
|
||||
)
|
||||
|
||||
type (
|
||||
garbageStore struct {
|
||||
*sync.RWMutex
|
||||
items []Address
|
||||
}
|
||||
)
|
||||
|
||||
func (s *garbageStore) put(addr Address) {
|
||||
s.Lock()
|
||||
defer s.Unlock()
|
||||
|
||||
for i := range s.items {
|
||||
if s.items[i].Equal(&addr) {
|
||||
return
|
||||
}
|
||||
}
|
||||
|
||||
s.items = append(s.items, addr)
|
||||
}
|
||||
|
||||
func newGarbageStore() *garbageStore { return &garbageStore{RWMutex: new(sync.RWMutex)} }
|
292
lib/replication/implementations.go
Normal file
292
lib/replication/implementations.go
Normal file
|
@ -0,0 +1,292 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"sync"
|
||||
|
||||
"github.com/multiformats/go-multiaddr"
|
||||
"github.com/nspcc-dev/neofs-node/internal"
|
||||
"github.com/nspcc-dev/neofs-node/lib/localstore"
|
||||
"github.com/nspcc-dev/neofs-node/lib/netmap"
|
||||
"github.com/nspcc-dev/neofs-node/lib/placement"
|
||||
"github.com/nspcc-dev/neofs-node/lib/rand"
|
||||
"github.com/pkg/errors"
|
||||
)
|
||||
|
||||
type (
|
||||
replicationScheduler struct {
|
||||
cac ContainerActualityChecker
|
||||
ls localstore.Iterator
|
||||
}
|
||||
|
||||
// SchedulerParams groups the parameters of scheduler constructor.
|
||||
SchedulerParams struct {
|
||||
ContainerActualityChecker
|
||||
localstore.Iterator
|
||||
}
|
||||
|
||||
objectPool struct {
|
||||
mu *sync.Mutex
|
||||
tasks []Address
|
||||
}
|
||||
|
||||
multiSolver struct {
|
||||
as AddressStore
|
||||
pl placement.Component
|
||||
}
|
||||
|
||||
// MultiSolverParams groups the parameters of multi solver constructor.
|
||||
MultiSolverParams struct {
|
||||
AddressStore
|
||||
Placement placement.Component
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
errPoolExhausted = internal.Error("object pool is exhausted")
|
||||
|
||||
objectPoolInstanceFailMsg = "could not create object pool"
|
||||
errEmptyLister = internal.Error("empty local objects lister")
|
||||
errEmptyContainerActual = internal.Error("empty container actuality checker")
|
||||
|
||||
multiSolverInstanceFailMsg = "could not create multi solver"
|
||||
errEmptyAddressStore = internal.Error("empty address store")
|
||||
errEmptyPlacement = internal.Error("empty placement")
|
||||
replicationSchedulerEntity = "replication scheduler"
|
||||
)
|
||||
|
||||
// NewObjectPool is an object pool constructor.
|
||||
func NewObjectPool() ObjectPool {
|
||||
return &objectPool{mu: new(sync.Mutex)}
|
||||
}
|
||||
|
||||
// NewReplicationScheduler is a replication scheduler constructor.
|
||||
func NewReplicationScheduler(p SchedulerParams) (Scheduler, error) {
|
||||
switch {
|
||||
case p.ContainerActualityChecker == nil:
|
||||
return nil, errors.Wrap(errEmptyContainerActual, objectPoolInstanceFailMsg)
|
||||
case p.Iterator == nil:
|
||||
return nil, errors.Wrap(errEmptyLister, objectPoolInstanceFailMsg)
|
||||
}
|
||||
|
||||
return &replicationScheduler{
|
||||
cac: p.ContainerActualityChecker,
|
||||
ls: p.Iterator,
|
||||
}, nil
|
||||
}
|
||||
|
||||
// NewMultiSolver is a multi solver constructor.
|
||||
func NewMultiSolver(p MultiSolverParams) (MultiSolver, error) {
|
||||
switch {
|
||||
case p.Placement == nil:
|
||||
return nil, errors.Wrap(errEmptyPlacement, multiSolverInstanceFailMsg)
|
||||
case p.AddressStore == nil:
|
||||
return nil, errors.Wrap(errEmptyAddressStore, multiSolverInstanceFailMsg)
|
||||
}
|
||||
|
||||
return &multiSolver{
|
||||
as: p.AddressStore,
|
||||
pl: p.Placement,
|
||||
}, nil
|
||||
}
|
||||
|
||||
func (s *objectPool) Update(pool []Address) {
|
||||
s.mu.Lock()
|
||||
defer s.mu.Unlock()
|
||||
|
||||
s.tasks = pool
|
||||
}
|
||||
|
||||
func (s *objectPool) Undone() int {
|
||||
s.mu.Lock()
|
||||
defer s.mu.Unlock()
|
||||
|
||||
return len(s.tasks)
|
||||
}
|
||||
|
||||
func (s *objectPool) Pop() (Address, error) {
|
||||
s.mu.Lock()
|
||||
defer s.mu.Unlock()
|
||||
|
||||
if len(s.tasks) == 0 {
|
||||
return Address{}, errPoolExhausted
|
||||
}
|
||||
|
||||
head := s.tasks[0]
|
||||
s.tasks = s.tasks[1:]
|
||||
|
||||
return head, nil
|
||||
}
|
||||
|
||||
func (s *replicationScheduler) SelectForReplication(limit int) ([]Address, error) {
|
||||
// Attention! This routine might be inefficient with big number of objects
|
||||
// and containers. Consider using fast traversal and filtering algorithms
|
||||
// with sieve of bloom filters.
|
||||
migration := make([]Address, 0, limit)
|
||||
replication := make([]Address, 0)
|
||||
ctx := context.Background()
|
||||
|
||||
if err := s.ls.Iterate(nil, func(meta *localstore.ObjectMeta) bool {
|
||||
if s.cac.Actual(ctx, meta.Object.SystemHeader.CID) {
|
||||
replication = append(replication, *meta.Object.Address())
|
||||
} else {
|
||||
migration = append(migration, *meta.Object.Address())
|
||||
}
|
||||
return len(migration) >= limit
|
||||
}); err != nil {
|
||||
return nil, err
|
||||
}
|
||||
|
||||
lnM := len(migration)
|
||||
lnR := len(replication)
|
||||
edge := 0
|
||||
|
||||
// I considered using rand.Perm() and appending elements in `for` cycle.
|
||||
// But it seems, that shuffling is efficient even when `limit-lnM`
|
||||
// is 1000 times smaller than `lnR`. But it can be discussed and changed
|
||||
// later anyway.
|
||||
if lnM < limit {
|
||||
r := rand.New()
|
||||
r.Shuffle(lnR, func(i, j int) {
|
||||
replication[i], replication[j] = replication[j], replication[i]
|
||||
})
|
||||
|
||||
edge = min(limit-lnM, lnR)
|
||||
}
|
||||
|
||||
return append(migration, replication[:edge]...), nil
|
||||
}
|
||||
|
||||
func (s *multiSolver) Epoch() uint64 { return s.pl.NetworkState().Epoch }
|
||||
|
||||
func (s *multiSolver) SelfAddr() (multiaddr.Multiaddr, error) { return s.as.SelfAddr() }
|
||||
func (s *multiSolver) ReservationRatio(ctx context.Context, addr Address) (int, error) {
|
||||
graph, err := s.pl.Query(ctx, placement.ContainerID(addr.CID))
|
||||
if err != nil {
|
||||
return 0, errors.Wrap(err, "reservation ratio computation failed on placement query")
|
||||
}
|
||||
|
||||
nodes, err := graph.Filter(func(group netmap.SFGroup, bucket *netmap.Bucket) *netmap.Bucket {
|
||||
return bucket.GetSelection(group.Selectors, addr.ObjectID.Bytes())
|
||||
}).NodeList()
|
||||
if err != nil {
|
||||
return 0, errors.Wrap(err, "reservation ratio computation failed on graph node list")
|
||||
}
|
||||
|
||||
return len(nodes), nil
|
||||
}
|
||||
|
||||
func (s *multiSolver) SelectRemoteStorages(ctx context.Context, addr Address, excl ...multiaddr.Multiaddr) ([]ObjectLocation, error) {
|
||||
selfAddr, err := s.as.SelfAddr()
|
||||
if err != nil {
|
||||
return nil, errors.Wrap(err, "select remote storage nodes failed on get self address")
|
||||
}
|
||||
|
||||
nodes, err := s.selectNodes(ctx, addr, excl...)
|
||||
if err != nil {
|
||||
return nil, errors.Wrap(err, "select remote storage nodes failed on get node list")
|
||||
}
|
||||
|
||||
var (
|
||||
metSelf bool
|
||||
selfIndex = -1
|
||||
res = make([]ObjectLocation, 0, len(nodes))
|
||||
)
|
||||
|
||||
for i := range nodes {
|
||||
if nodes[i].Equal(selfAddr) {
|
||||
metSelf = true
|
||||
selfIndex = i
|
||||
}
|
||||
|
||||
res = append(res, ObjectLocation{
|
||||
Node: nodes[i],
|
||||
WeightGreater: !metSelf,
|
||||
})
|
||||
}
|
||||
|
||||
if selfIndex != -1 {
|
||||
res = append(res[:selfIndex], res[selfIndex+1:]...)
|
||||
}
|
||||
|
||||
return res, nil
|
||||
}
|
||||
|
||||
func (s *multiSolver) selectNodes(ctx context.Context, addr Address, excl ...multiaddr.Multiaddr) ([]multiaddr.Multiaddr, error) {
|
||||
graph, err := s.pl.Query(ctx, placement.ContainerID(addr.CID))
|
||||
if err != nil {
|
||||
return nil, errors.Wrap(err, "select remote storage nodes failed on placement query")
|
||||
}
|
||||
|
||||
filter := func(group netmap.SFGroup, bucket *netmap.Bucket) *netmap.Bucket { return bucket }
|
||||
if !addr.ObjectID.Empty() {
|
||||
filter = func(group netmap.SFGroup, bucket *netmap.Bucket) *netmap.Bucket {
|
||||
return bucket.GetSelection(group.Selectors, addr.ObjectID.Bytes())
|
||||
}
|
||||
}
|
||||
|
||||
return graph.Exclude(excl).Filter(filter).NodeList()
|
||||
}
|
||||
|
||||
func (s *multiSolver) Actual(ctx context.Context, cid CID) bool {
|
||||
graph, err := s.pl.Query(ctx, placement.ContainerID(cid))
|
||||
if err != nil {
|
||||
return false
|
||||
}
|
||||
|
||||
nodes, err := graph.NodeList()
|
||||
if err != nil {
|
||||
return false
|
||||
}
|
||||
|
||||
selfAddr, err := s.as.SelfAddr()
|
||||
if err != nil {
|
||||
return false
|
||||
}
|
||||
|
||||
for i := range nodes {
|
||||
if nodes[i].Equal(selfAddr) {
|
||||
return true
|
||||
}
|
||||
}
|
||||
|
||||
return false
|
||||
}
|
||||
|
||||
func (s *multiSolver) CompareWeight(ctx context.Context, addr Address, node multiaddr.Multiaddr) int {
|
||||
selfAddr, err := s.as.SelfAddr()
|
||||
if err != nil {
|
||||
return -1
|
||||
}
|
||||
|
||||
if selfAddr.Equal(node) {
|
||||
return 0
|
||||
}
|
||||
|
||||
excl := make([]multiaddr.Multiaddr, 0)
|
||||
|
||||
for {
|
||||
nodes, err := s.selectNodes(ctx, addr, excl...)
|
||||
if err != nil {
|
||||
return -1
|
||||
}
|
||||
|
||||
for j := range nodes {
|
||||
if nodes[j].Equal(selfAddr) {
|
||||
return -1
|
||||
} else if nodes[j].Equal(node) {
|
||||
return 1
|
||||
}
|
||||
}
|
||||
|
||||
excl = append(excl, nodes[0]) // TODO: when it will become relevant to append full nodes slice
|
||||
}
|
||||
}
|
||||
|
||||
func min(a, b int) int {
|
||||
if a < b {
|
||||
return a
|
||||
}
|
||||
|
||||
return b
|
||||
}
|
154
lib/replication/location_detector.go
Normal file
154
lib/replication/location_detector.go
Normal file
|
@ -0,0 +1,154 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// ObjectLocationDetector is an interface of entity
|
||||
// that listens tasks to detect object current locations in network.
|
||||
ObjectLocationDetector interface {
|
||||
Process(ctx context.Context) chan<- Address
|
||||
Subscribe(ch chan<- *ObjectLocationRecord)
|
||||
}
|
||||
|
||||
objectLocationDetector struct {
|
||||
weightComparator WeightComparator
|
||||
objectLocator ObjectLocator
|
||||
reservationRatioReceiver ReservationRatioReceiver
|
||||
presenceChecker PresenceChecker
|
||||
log *zap.Logger
|
||||
|
||||
taskChanCap int
|
||||
resultTimeout time.Duration
|
||||
resultChan chan<- *ObjectLocationRecord
|
||||
}
|
||||
|
||||
// LocationDetectorParams groups the parameters of location detector's constructor.
|
||||
LocationDetectorParams struct {
|
||||
WeightComparator
|
||||
ObjectLocator
|
||||
ReservationRatioReceiver
|
||||
PresenceChecker
|
||||
*zap.Logger
|
||||
|
||||
TaskChanCap int
|
||||
ResultTimeout time.Duration
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
defaultLocationDetectorChanCap = 10
|
||||
defaultLocationDetectorResultTimeout = time.Second
|
||||
locationDetectorEntity = "object location detector"
|
||||
)
|
||||
|
||||
func (s *objectLocationDetector) Subscribe(ch chan<- *ObjectLocationRecord) { s.resultChan = ch }
|
||||
|
||||
func (s *objectLocationDetector) Process(ctx context.Context) chan<- Address {
|
||||
ch := make(chan Address, s.taskChanCap)
|
||||
go s.processRoutine(ctx, ch)
|
||||
|
||||
return ch
|
||||
}
|
||||
|
||||
func (s *objectLocationDetector) writeResult(locationRecord *ObjectLocationRecord) {
|
||||
if s.resultChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.resultChan <- locationRecord:
|
||||
case <-time.After(s.resultTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *objectLocationDetector) processRoutine(ctx context.Context, taskChan <-chan Address) {
|
||||
loop:
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(locationDetectorEntity+ctxDoneMsg, zap.Error(ctx.Err()))
|
||||
break loop
|
||||
case addr, ok := <-taskChan:
|
||||
if !ok {
|
||||
s.log.Warn(locationDetectorEntity + taskChanClosed)
|
||||
break loop
|
||||
} else if has, err := s.presenceChecker.Has(addr); err != nil || !has {
|
||||
continue loop
|
||||
}
|
||||
s.handleTask(ctx, addr)
|
||||
}
|
||||
}
|
||||
close(s.resultChan)
|
||||
}
|
||||
|
||||
func (s *objectLocationDetector) handleTask(ctx context.Context, addr Address) {
|
||||
var (
|
||||
err error
|
||||
log = s.log.With(addressFields(addr)...)
|
||||
locationRecord = &ObjectLocationRecord{addr, 0, nil}
|
||||
)
|
||||
|
||||
if locationRecord.ReservationRatio, err = s.reservationRatioReceiver.ReservationRatio(ctx, addr); err != nil {
|
||||
log.Error("reservation ratio computation failure", zap.Error(err))
|
||||
return
|
||||
}
|
||||
|
||||
nodes, err := s.objectLocator.LocateObject(ctx, addr)
|
||||
if err != nil {
|
||||
log.Error("locate object failure", zap.Error(err))
|
||||
return
|
||||
}
|
||||
|
||||
for i := range nodes {
|
||||
locationRecord.Locations = append(locationRecord.Locations, ObjectLocation{
|
||||
Node: nodes[i],
|
||||
WeightGreater: s.weightComparator.CompareWeight(ctx, addr, nodes[i]) == 1,
|
||||
})
|
||||
}
|
||||
|
||||
log.Debug("current location record created",
|
||||
zap.Int("reservation ratio", locationRecord.ReservationRatio),
|
||||
zap.Any("storage nodes exclude self", locationRecord.Locations))
|
||||
|
||||
s.writeResult(locationRecord)
|
||||
}
|
||||
|
||||
// NewLocationDetector is an object location detector's constructor.
|
||||
func NewLocationDetector(p *LocationDetectorParams) (ObjectLocationDetector, error) {
|
||||
switch {
|
||||
case p.PresenceChecker == nil:
|
||||
return nil, instanceError(locationDetectorEntity, presenceCheckerPart)
|
||||
case p.ObjectLocator == nil:
|
||||
return nil, instanceError(locationDetectorEntity, objectLocatorPart)
|
||||
case p.ReservationRatioReceiver == nil:
|
||||
return nil, instanceError(locationDetectorEntity, reservationRatioReceiverPart)
|
||||
case p.Logger == nil:
|
||||
return nil, instanceError(locationDetectorEntity, loggerPart)
|
||||
case p.WeightComparator == nil:
|
||||
return nil, instanceError(locationDetectorEntity, weightComparatorPart)
|
||||
}
|
||||
|
||||
if p.TaskChanCap <= 0 {
|
||||
p.TaskChanCap = defaultLocationDetectorChanCap
|
||||
}
|
||||
|
||||
if p.ResultTimeout <= 0 {
|
||||
p.ResultTimeout = defaultLocationDetectorResultTimeout
|
||||
}
|
||||
|
||||
return &objectLocationDetector{
|
||||
weightComparator: p.WeightComparator,
|
||||
objectLocator: p.ObjectLocator,
|
||||
reservationRatioReceiver: p.ReservationRatioReceiver,
|
||||
presenceChecker: p.PresenceChecker,
|
||||
log: p.Logger,
|
||||
taskChanCap: p.TaskChanCap,
|
||||
resultTimeout: p.ResultTimeout,
|
||||
resultChan: nil,
|
||||
}, nil
|
||||
}
|
347
lib/replication/manager.go
Normal file
347
lib/replication/manager.go
Normal file
|
@ -0,0 +1,347 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"fmt"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// Manager is an interface of object manager,
|
||||
Manager interface {
|
||||
Process(ctx context.Context)
|
||||
HandleEpoch(ctx context.Context, epoch uint64)
|
||||
}
|
||||
|
||||
manager struct {
|
||||
objectPool ObjectPool
|
||||
managerTimeout time.Duration
|
||||
objectVerifier ObjectVerifier
|
||||
log *zap.Logger
|
||||
|
||||
locationDetector ObjectLocationDetector
|
||||
storageValidator StorageValidator
|
||||
replicator ObjectReplicator
|
||||
restorer ObjectRestorer
|
||||
placementHonorer PlacementHonorer
|
||||
|
||||
// internal task channels
|
||||
detectLocationTaskChan chan<- Address
|
||||
restoreTaskChan chan<- Address
|
||||
|
||||
pushTaskTimeout time.Duration
|
||||
|
||||
// internal result channels
|
||||
replicationResultChan <-chan *ReplicateResult
|
||||
restoreResultChan <-chan Address
|
||||
|
||||
garbageChanCap int
|
||||
replicateResultChanCap int
|
||||
restoreResultChanCap int
|
||||
|
||||
garbageChan <-chan Address
|
||||
garbageStore *garbageStore
|
||||
|
||||
epochCh chan uint64
|
||||
scheduler Scheduler
|
||||
|
||||
poolSize int
|
||||
poolExpansionRate float64
|
||||
}
|
||||
|
||||
// ManagerParams groups the parameters of object manager's constructor.
|
||||
ManagerParams struct {
|
||||
Interval time.Duration
|
||||
PushTaskTimeout time.Duration
|
||||
PlacementHonorerEnabled bool
|
||||
ReplicateTaskChanCap int
|
||||
RestoreTaskChanCap int
|
||||
GarbageChanCap int
|
||||
InitPoolSize int
|
||||
ExpansionRate float64
|
||||
|
||||
ObjectPool
|
||||
ObjectVerifier
|
||||
|
||||
PlacementHonorer
|
||||
ObjectLocationDetector
|
||||
StorageValidator
|
||||
ObjectReplicator
|
||||
ObjectRestorer
|
||||
|
||||
*zap.Logger
|
||||
|
||||
Scheduler
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
managerEntity = "replication manager"
|
||||
|
||||
redundantCopiesBeagleName = "BEAGLE_REDUNDANT_COPIES"
|
||||
|
||||
defaultInterval = 3 * time.Second
|
||||
defaultPushTaskTimeout = time.Second
|
||||
|
||||
defaultGarbageChanCap = 10
|
||||
defaultReplicateResultChanCap = 10
|
||||
defaultRestoreResultChanCap = 10
|
||||
)
|
||||
|
||||
func (s *manager) Name() string { return redundantCopiesBeagleName }
|
||||
|
||||
func (s *manager) HandleEpoch(ctx context.Context, epoch uint64) {
|
||||
select {
|
||||
case s.epochCh <- epoch:
|
||||
case <-ctx.Done():
|
||||
return
|
||||
case <-time.After(s.managerTimeout):
|
||||
// this timeout must never happen
|
||||
// if timeout happens in runtime, then something is definitely wrong!
|
||||
s.log.Warn("replication scheduler is busy")
|
||||
}
|
||||
}
|
||||
|
||||
func (s *manager) Process(ctx context.Context) {
|
||||
// starting object restorer
|
||||
// bind manager to push restore tasks to restorer
|
||||
s.restoreTaskChan = s.restorer.Process(ctx)
|
||||
|
||||
// bind manager to listen object restorer results
|
||||
restoreResultChan := make(chan Address, s.restoreResultChanCap)
|
||||
s.restoreResultChan = restoreResultChan
|
||||
s.restorer.Subscribe(restoreResultChan)
|
||||
|
||||
// starting location detector
|
||||
// bind manager to push locate tasks to location detector
|
||||
s.detectLocationTaskChan = s.locationDetector.Process(ctx)
|
||||
|
||||
locationsHandlerStartFn := s.storageValidator.Process
|
||||
if s.placementHonorer != nil {
|
||||
locationsHandlerStartFn = s.placementHonorer.Process
|
||||
|
||||
// starting storage validator
|
||||
// bind placement honorer to push validate tasks to storage validator
|
||||
s.placementHonorer.Subscribe(s.storageValidator.Process(ctx))
|
||||
}
|
||||
|
||||
// starting location handler component
|
||||
// bind location detector to push tasks to location handler component
|
||||
s.locationDetector.Subscribe(locationsHandlerStartFn(ctx))
|
||||
|
||||
// bind manager to listen object replicator results
|
||||
replicateResultChan := make(chan *ReplicateResult, s.replicateResultChanCap)
|
||||
s.replicationResultChan = replicateResultChan
|
||||
s.replicator.Subscribe(replicateResultChan)
|
||||
|
||||
// starting replicator
|
||||
// bind storage validator to push replicate tasks to replicator
|
||||
s.storageValidator.SubscribeReplication(s.replicator.Process(ctx))
|
||||
garbageChan := make(chan Address, s.garbageChanCap)
|
||||
s.garbageChan = garbageChan
|
||||
s.storageValidator.SubscribeGarbage(garbageChan)
|
||||
|
||||
go s.taskRoutine(ctx)
|
||||
go s.resultRoutine(ctx)
|
||||
s.processRoutine(ctx)
|
||||
}
|
||||
|
||||
func resultLog(s1, s2 string) string {
|
||||
return fmt.Sprintf(managerEntity+" %s process finish: %s", s1, s2)
|
||||
}
|
||||
|
||||
func (s *manager) writeDetectLocationTask(addr Address) {
|
||||
if s.detectLocationTaskChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.detectLocationTaskChan <- addr:
|
||||
case <-time.After(s.pushTaskTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *manager) writeRestoreTask(addr Address) {
|
||||
if s.restoreTaskChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.restoreTaskChan <- addr:
|
||||
case <-time.After(s.pushTaskTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *manager) resultRoutine(ctx context.Context) {
|
||||
loop:
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(resultLog("result", ctxDoneMsg), zap.Error(ctx.Err()))
|
||||
break loop
|
||||
case addr, ok := <-s.restoreResultChan:
|
||||
if !ok {
|
||||
s.log.Warn(resultLog("result", "restorer result channel closed"))
|
||||
break loop
|
||||
}
|
||||
s.log.Info("object successfully restored", addressFields(addr)...)
|
||||
case res, ok := <-s.replicationResultChan:
|
||||
if !ok {
|
||||
s.log.Warn(resultLog("result", "replicator result channel closed"))
|
||||
break loop
|
||||
} else if len(res.NewStorages) > 0 {
|
||||
s.log.Info("object successfully replicated",
|
||||
append(addressFields(res.Address), zap.Any("new storages", res.NewStorages))...)
|
||||
}
|
||||
case addr, ok := <-s.garbageChan:
|
||||
if !ok {
|
||||
s.log.Warn(resultLog("result", "garbage channel closed"))
|
||||
break loop
|
||||
}
|
||||
s.garbageStore.put(addr)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
func (s *manager) taskRoutine(ctx context.Context) {
|
||||
loop:
|
||||
for {
|
||||
if task, err := s.objectPool.Pop(); err == nil {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(resultLog("task", ctxDoneMsg), zap.Error(ctx.Err()))
|
||||
break loop
|
||||
default:
|
||||
s.distributeTask(ctx, task)
|
||||
}
|
||||
} else {
|
||||
// if object pool is empty, check it again after a while
|
||||
time.Sleep(s.managerTimeout)
|
||||
}
|
||||
}
|
||||
close(s.restoreTaskChan)
|
||||
close(s.detectLocationTaskChan)
|
||||
}
|
||||
|
||||
func (s *manager) processRoutine(ctx context.Context) {
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
return
|
||||
case epoch := <-s.epochCh:
|
||||
var delta int
|
||||
|
||||
// undone - amount of objects we couldn't process in last epoch
|
||||
undone := s.objectPool.Undone()
|
||||
if undone > 0 {
|
||||
// if there are unprocessed objects, then lower your estimation
|
||||
delta = -undone
|
||||
} else {
|
||||
// otherwise try to expand
|
||||
delta = int(float64(s.poolSize) * s.poolExpansionRate)
|
||||
}
|
||||
|
||||
tasks, err := s.scheduler.SelectForReplication(s.poolSize + delta)
|
||||
if err != nil {
|
||||
s.log.Warn("can't select objects for replication", zap.Error(err))
|
||||
}
|
||||
|
||||
// if there are NOT enough objects to fill the pool, do not change it
|
||||
// otherwise expand or shrink it with the delta value
|
||||
if len(tasks) >= s.poolSize+delta {
|
||||
s.poolSize += delta
|
||||
}
|
||||
|
||||
s.objectPool.Update(tasks)
|
||||
|
||||
s.log.Info("replication schedule updated",
|
||||
zap.Int("unprocessed_tasks", undone),
|
||||
zap.Int("next_tasks", len(tasks)),
|
||||
zap.Int("pool_size", s.poolSize),
|
||||
zap.Uint64("new_epoch", epoch))
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
// Function takes object from storage by address (if verify
|
||||
// If verify flag is set object stored incorrectly (Verify returned error) - restore task is planned
|
||||
// otherwise validate task is planned.
|
||||
func (s *manager) distributeTask(ctx context.Context, addr Address) {
|
||||
if !s.objectVerifier.Verify(ctx, &ObjectVerificationParams{Address: addr}) {
|
||||
s.writeRestoreTask(addr)
|
||||
return
|
||||
}
|
||||
|
||||
s.writeDetectLocationTask(addr)
|
||||
}
|
||||
|
||||
// NewManager is an object manager's constructor.
|
||||
func NewManager(p ManagerParams) (Manager, error) {
|
||||
switch {
|
||||
case p.ObjectPool == nil:
|
||||
return nil, instanceError(managerEntity, objectPoolPart)
|
||||
case p.ObjectVerifier == nil:
|
||||
return nil, instanceError(managerEntity, objectVerifierPart)
|
||||
case p.Logger == nil:
|
||||
return nil, instanceError(managerEntity, loggerPart)
|
||||
case p.ObjectLocationDetector == nil:
|
||||
return nil, instanceError(managerEntity, locationDetectorEntity)
|
||||
case p.StorageValidator == nil:
|
||||
return nil, instanceError(managerEntity, storageValidatorEntity)
|
||||
case p.ObjectReplicator == nil:
|
||||
return nil, instanceError(managerEntity, objectReplicatorEntity)
|
||||
case p.ObjectRestorer == nil:
|
||||
return nil, instanceError(managerEntity, objectRestorerEntity)
|
||||
case p.PlacementHonorer == nil && p.PlacementHonorerEnabled:
|
||||
return nil, instanceError(managerEntity, placementHonorerEntity)
|
||||
case p.Scheduler == nil:
|
||||
return nil, instanceError(managerEntity, replicationSchedulerEntity)
|
||||
}
|
||||
|
||||
if p.Interval <= 0 {
|
||||
p.Interval = defaultInterval
|
||||
}
|
||||
|
||||
if p.PushTaskTimeout <= 0 {
|
||||
p.PushTaskTimeout = defaultPushTaskTimeout
|
||||
}
|
||||
|
||||
if p.GarbageChanCap <= 0 {
|
||||
p.GarbageChanCap = defaultGarbageChanCap
|
||||
}
|
||||
|
||||
if p.ReplicateTaskChanCap <= 0 {
|
||||
p.ReplicateTaskChanCap = defaultReplicateResultChanCap
|
||||
}
|
||||
|
||||
if p.RestoreTaskChanCap <= 0 {
|
||||
p.RestoreTaskChanCap = defaultRestoreResultChanCap
|
||||
}
|
||||
|
||||
if !p.PlacementHonorerEnabled {
|
||||
p.PlacementHonorer = nil
|
||||
}
|
||||
|
||||
return &manager{
|
||||
objectPool: p.ObjectPool,
|
||||
managerTimeout: p.Interval,
|
||||
objectVerifier: p.ObjectVerifier,
|
||||
log: p.Logger,
|
||||
locationDetector: p.ObjectLocationDetector,
|
||||
storageValidator: p.StorageValidator,
|
||||
replicator: p.ObjectReplicator,
|
||||
restorer: p.ObjectRestorer,
|
||||
placementHonorer: p.PlacementHonorer,
|
||||
pushTaskTimeout: p.PushTaskTimeout,
|
||||
garbageChanCap: p.GarbageChanCap,
|
||||
replicateResultChanCap: p.ReplicateTaskChanCap,
|
||||
restoreResultChanCap: p.RestoreTaskChanCap,
|
||||
garbageStore: newGarbageStore(),
|
||||
epochCh: make(chan uint64),
|
||||
scheduler: p.Scheduler,
|
||||
poolSize: p.InitPoolSize,
|
||||
poolExpansionRate: p.ExpansionRate,
|
||||
}, nil
|
||||
}
|
188
lib/replication/object_replicator.go
Normal file
188
lib/replication/object_replicator.go
Normal file
|
@ -0,0 +1,188 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"time"
|
||||
|
||||
"github.com/multiformats/go-multiaddr"
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// ObjectReplicator is an interface of entity
|
||||
// that listens object replication tasks.
|
||||
// Result includes new object storage list.
|
||||
ObjectReplicator interface {
|
||||
Process(ctx context.Context) chan<- *ReplicateTask
|
||||
Subscribe(ch chan<- *ReplicateResult)
|
||||
}
|
||||
|
||||
objectReplicator struct {
|
||||
objectReceptacle ObjectReceptacle
|
||||
remoteStorageSelector RemoteStorageSelector
|
||||
objectSource ObjectSource
|
||||
presenceChecker PresenceChecker
|
||||
log *zap.Logger
|
||||
|
||||
taskChanCap int
|
||||
resultTimeout time.Duration
|
||||
resultChan chan<- *ReplicateResult
|
||||
}
|
||||
|
||||
// ObjectReplicatorParams groups the parameters of replicator's constructor.
|
||||
ObjectReplicatorParams struct {
|
||||
RemoteStorageSelector
|
||||
ObjectSource
|
||||
ObjectReceptacle
|
||||
PresenceChecker
|
||||
*zap.Logger
|
||||
|
||||
TaskChanCap int
|
||||
ResultTimeout time.Duration
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
defaultReplicatorChanCap = 10
|
||||
defaultReplicatorResultTimeout = time.Second
|
||||
objectReplicatorEntity = "object replicator"
|
||||
)
|
||||
|
||||
func (s *objectReplicator) Subscribe(ch chan<- *ReplicateResult) { s.resultChan = ch }
|
||||
|
||||
func (s *objectReplicator) Process(ctx context.Context) chan<- *ReplicateTask {
|
||||
ch := make(chan *ReplicateTask, s.taskChanCap)
|
||||
go s.processRoutine(ctx, ch)
|
||||
|
||||
return ch
|
||||
}
|
||||
|
||||
func (s *objectReplicator) writeResult(replicateResult *ReplicateResult) {
|
||||
if s.resultChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.resultChan <- replicateResult:
|
||||
case <-time.After(s.resultTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *objectReplicator) processRoutine(ctx context.Context, taskChan <-chan *ReplicateTask) {
|
||||
loop:
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(objectReplicatorEntity+" process finish: context completed",
|
||||
zap.Error(ctx.Err()))
|
||||
break loop
|
||||
case replicateTask, ok := <-taskChan:
|
||||
if !ok {
|
||||
s.log.Warn(objectReplicatorEntity + " process finish: task channel closed")
|
||||
break loop
|
||||
} else if has, err := s.presenceChecker.Has(replicateTask.Address); err != nil || !has {
|
||||
continue loop
|
||||
}
|
||||
s.handleTask(ctx, replicateTask)
|
||||
}
|
||||
}
|
||||
close(s.resultChan)
|
||||
}
|
||||
|
||||
func (s *objectReplicator) handleTask(ctx context.Context, task *ReplicateTask) {
|
||||
obj, err := s.objectSource.Get(ctx, task.Address)
|
||||
if err != nil {
|
||||
s.log.Warn("get object from storage failure", zap.Error(err))
|
||||
return
|
||||
}
|
||||
|
||||
res := &ReplicateResult{
|
||||
ReplicateTask: task,
|
||||
NewStorages: make([]multiaddr.Multiaddr, 0, task.Shortage),
|
||||
}
|
||||
|
||||
for len(res.NewStorages) < task.Shortage {
|
||||
nodesInfo, err := s.remoteStorageSelector.SelectRemoteStorages(ctx, task.Address, task.ExcludeNodes...)
|
||||
if err != nil {
|
||||
break
|
||||
}
|
||||
|
||||
for i := 0; i < len(nodesInfo); i++ {
|
||||
if contains(res.NewStorages, nodesInfo[i].Node) {
|
||||
nodesInfo = append(nodesInfo[:i], nodesInfo[i+1:]...)
|
||||
i--
|
||||
|
||||
continue
|
||||
}
|
||||
}
|
||||
|
||||
if len(nodesInfo) > task.Shortage {
|
||||
nodesInfo = nodesInfo[:task.Shortage]
|
||||
}
|
||||
|
||||
if len(nodesInfo) == 0 {
|
||||
break
|
||||
}
|
||||
|
||||
if err := s.objectReceptacle.Put(ctx, ObjectStoreParams{
|
||||
Object: obj,
|
||||
Nodes: nodesInfo,
|
||||
Handler: func(location ObjectLocation, success bool) {
|
||||
if success {
|
||||
res.NewStorages = append(res.NewStorages, location.Node)
|
||||
} else {
|
||||
task.ExcludeNodes = append(task.ExcludeNodes, location.Node)
|
||||
}
|
||||
},
|
||||
}); err != nil {
|
||||
s.log.Warn("replicate object failure", zap.Error(err))
|
||||
break
|
||||
}
|
||||
}
|
||||
|
||||
s.writeResult(res)
|
||||
}
|
||||
|
||||
func contains(list []multiaddr.Multiaddr, item multiaddr.Multiaddr) bool {
|
||||
for i := range list {
|
||||
if list[i].Equal(item) {
|
||||
return true
|
||||
}
|
||||
}
|
||||
|
||||
return false
|
||||
}
|
||||
|
||||
// NewReplicator is an object replicator's constructor.
|
||||
func NewReplicator(p ObjectReplicatorParams) (ObjectReplicator, error) {
|
||||
switch {
|
||||
case p.ObjectReceptacle == nil:
|
||||
return nil, instanceError(objectReplicatorEntity, objectReceptaclePart)
|
||||
case p.ObjectSource == nil:
|
||||
return nil, instanceError(objectReplicatorEntity, objectSourcePart)
|
||||
case p.RemoteStorageSelector == nil:
|
||||
return nil, instanceError(objectReplicatorEntity, remoteStorageSelectorPart)
|
||||
case p.PresenceChecker == nil:
|
||||
return nil, instanceError(objectReplicatorEntity, presenceCheckerPart)
|
||||
case p.Logger == nil:
|
||||
return nil, instanceError(objectReplicatorEntity, loggerPart)
|
||||
}
|
||||
|
||||
if p.TaskChanCap <= 0 {
|
||||
p.TaskChanCap = defaultReplicatorChanCap
|
||||
}
|
||||
|
||||
if p.ResultTimeout <= 0 {
|
||||
p.ResultTimeout = defaultReplicatorResultTimeout
|
||||
}
|
||||
|
||||
return &objectReplicator{
|
||||
objectReceptacle: p.ObjectReceptacle,
|
||||
remoteStorageSelector: p.RemoteStorageSelector,
|
||||
objectSource: p.ObjectSource,
|
||||
presenceChecker: p.PresenceChecker,
|
||||
log: p.Logger,
|
||||
taskChanCap: p.TaskChanCap,
|
||||
resultTimeout: p.ResultTimeout,
|
||||
}, nil
|
||||
}
|
173
lib/replication/object_restorer.go
Normal file
173
lib/replication/object_restorer.go
Normal file
|
@ -0,0 +1,173 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"time"
|
||||
|
||||
"github.com/multiformats/go-multiaddr"
|
||||
"github.com/nspcc-dev/neofs-node/lib/localstore"
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// ObjectRestorer is an interface of entity
|
||||
// that listen tasks to restore object by address.
|
||||
// Restorer doesn't recheck if object is actually corrupted.
|
||||
// Restorer writes result to subscriber only if restoration was successful.
|
||||
ObjectRestorer interface {
|
||||
Process(ctx context.Context) chan<- Address
|
||||
Subscribe(ch chan<- Address)
|
||||
}
|
||||
|
||||
objectRestorer struct {
|
||||
objectVerifier ObjectVerifier
|
||||
remoteStorageSelector RemoteStorageSelector
|
||||
objectReceptacle ObjectReceptacle
|
||||
epochReceiver EpochReceiver
|
||||
presenceChecker PresenceChecker
|
||||
log *zap.Logger
|
||||
|
||||
taskChanCap int
|
||||
resultTimeout time.Duration
|
||||
resultChan chan<- Address
|
||||
}
|
||||
|
||||
// ObjectRestorerParams groups the parameters of object restorer's constructor.
|
||||
ObjectRestorerParams struct {
|
||||
ObjectVerifier
|
||||
ObjectReceptacle
|
||||
EpochReceiver
|
||||
RemoteStorageSelector
|
||||
PresenceChecker
|
||||
*zap.Logger
|
||||
|
||||
TaskChanCap int
|
||||
ResultTimeout time.Duration
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
defaultRestorerChanCap = 10
|
||||
defaultRestorerResultTimeout = time.Second
|
||||
objectRestorerEntity = "object restorer"
|
||||
)
|
||||
|
||||
func (s *objectRestorer) Subscribe(ch chan<- Address) { s.resultChan = ch }
|
||||
|
||||
func (s *objectRestorer) Process(ctx context.Context) chan<- Address {
|
||||
ch := make(chan Address, s.taskChanCap)
|
||||
go s.processRoutine(ctx, ch)
|
||||
|
||||
return ch
|
||||
}
|
||||
|
||||
func (s *objectRestorer) writeResult(refInfo Address) {
|
||||
if s.resultChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.resultChan <- refInfo:
|
||||
case <-time.After(s.resultTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *objectRestorer) processRoutine(ctx context.Context, taskChan <-chan Address) {
|
||||
loop:
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(objectRestorerEntity+ctxDoneMsg, zap.Error(ctx.Err()))
|
||||
break loop
|
||||
case addr, ok := <-taskChan:
|
||||
if !ok {
|
||||
s.log.Warn(objectRestorerEntity + taskChanClosed)
|
||||
break loop
|
||||
} else if has, err := s.presenceChecker.Has(addr); err != nil || !has {
|
||||
continue loop
|
||||
}
|
||||
s.handleTask(ctx, addr)
|
||||
}
|
||||
}
|
||||
close(s.resultChan)
|
||||
}
|
||||
|
||||
func (s *objectRestorer) handleTask(ctx context.Context, addr Address) {
|
||||
var (
|
||||
receivedObj *Object
|
||||
exclNodes = make([]multiaddr.Multiaddr, 0)
|
||||
)
|
||||
|
||||
loop:
|
||||
for {
|
||||
nodesInfo, err := s.remoteStorageSelector.SelectRemoteStorages(ctx, addr, exclNodes...)
|
||||
if err != nil {
|
||||
break
|
||||
}
|
||||
|
||||
for i := range nodesInfo {
|
||||
info := nodesInfo[i]
|
||||
if s.objectVerifier.Verify(ctx, &ObjectVerificationParams{
|
||||
Address: addr,
|
||||
Node: nodesInfo[i].Node,
|
||||
Handler: func(valid bool, obj *Object) {
|
||||
if valid {
|
||||
receivedObj = obj
|
||||
} else {
|
||||
exclNodes = append(exclNodes, info.Node)
|
||||
}
|
||||
},
|
||||
LocalInvalid: true,
|
||||
}) {
|
||||
break loop
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
if err := s.objectReceptacle.Put(
|
||||
context.WithValue(ctx, localstore.StoreEpochValue, s.epochReceiver.Epoch()),
|
||||
ObjectStoreParams{Object: receivedObj},
|
||||
); err != nil {
|
||||
s.log.Warn("put object to local storage failure", append(addressFields(addr), zap.Error(err))...)
|
||||
return
|
||||
}
|
||||
|
||||
s.writeResult(addr)
|
||||
}
|
||||
|
||||
// NewObjectRestorer is an object restorer's constructor.
|
||||
func NewObjectRestorer(p *ObjectRestorerParams) (ObjectRestorer, error) {
|
||||
switch {
|
||||
case p.Logger == nil:
|
||||
return nil, instanceError(objectRestorerEntity, loggerPart)
|
||||
case p.ObjectVerifier == nil:
|
||||
return nil, instanceError(objectRestorerEntity, objectVerifierPart)
|
||||
case p.ObjectReceptacle == nil:
|
||||
return nil, instanceError(objectRestorerEntity, objectReceptaclePart)
|
||||
case p.RemoteStorageSelector == nil:
|
||||
return nil, instanceError(objectRestorerEntity, remoteStorageSelectorPart)
|
||||
case p.EpochReceiver == nil:
|
||||
return nil, instanceError(objectRestorerEntity, epochReceiverPart)
|
||||
case p.PresenceChecker == nil:
|
||||
return nil, instanceError(objectRestorerEntity, presenceCheckerPart)
|
||||
}
|
||||
|
||||
if p.TaskChanCap <= 0 {
|
||||
p.TaskChanCap = defaultRestorerChanCap
|
||||
}
|
||||
|
||||
if p.ResultTimeout <= 0 {
|
||||
p.ResultTimeout = defaultRestorerResultTimeout
|
||||
}
|
||||
|
||||
return &objectRestorer{
|
||||
objectVerifier: p.ObjectVerifier,
|
||||
remoteStorageSelector: p.RemoteStorageSelector,
|
||||
objectReceptacle: p.ObjectReceptacle,
|
||||
epochReceiver: p.EpochReceiver,
|
||||
presenceChecker: p.PresenceChecker,
|
||||
log: p.Logger,
|
||||
taskChanCap: p.TaskChanCap,
|
||||
resultTimeout: p.ResultTimeout,
|
||||
}, nil
|
||||
}
|
198
lib/replication/placement_honorer.go
Normal file
198
lib/replication/placement_honorer.go
Normal file
|
@ -0,0 +1,198 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"time"
|
||||
|
||||
"github.com/multiformats/go-multiaddr"
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// PlacementHonorer is an interface of entity
|
||||
// that listens tasks to piece out placement rule of container for particular object.
|
||||
PlacementHonorer interface {
|
||||
Process(ctx context.Context) chan<- *ObjectLocationRecord
|
||||
Subscribe(ch chan<- *ObjectLocationRecord)
|
||||
}
|
||||
|
||||
placementHonorer struct {
|
||||
objectSource ObjectSource
|
||||
objectReceptacle ObjectReceptacle
|
||||
remoteStorageSelector RemoteStorageSelector
|
||||
presenceChecker PresenceChecker
|
||||
log *zap.Logger
|
||||
|
||||
taskChanCap int
|
||||
resultTimeout time.Duration
|
||||
resultChan chan<- *ObjectLocationRecord
|
||||
}
|
||||
|
||||
// PlacementHonorerParams groups the parameters of placement honorer's constructor.
|
||||
PlacementHonorerParams struct {
|
||||
ObjectSource
|
||||
ObjectReceptacle
|
||||
RemoteStorageSelector
|
||||
PresenceChecker
|
||||
*zap.Logger
|
||||
|
||||
TaskChanCap int
|
||||
ResultTimeout time.Duration
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
defaultPlacementHonorerChanCap = 10
|
||||
defaultPlacementHonorerResultTimeout = time.Second
|
||||
placementHonorerEntity = "placement honorer"
|
||||
)
|
||||
|
||||
func (s *placementHonorer) Subscribe(ch chan<- *ObjectLocationRecord) { s.resultChan = ch }
|
||||
|
||||
func (s *placementHonorer) Process(ctx context.Context) chan<- *ObjectLocationRecord {
|
||||
ch := make(chan *ObjectLocationRecord, s.taskChanCap)
|
||||
go s.processRoutine(ctx, ch)
|
||||
|
||||
return ch
|
||||
}
|
||||
|
||||
func (s *placementHonorer) writeResult(locationRecord *ObjectLocationRecord) {
|
||||
if s.resultChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.resultChan <- locationRecord:
|
||||
case <-time.After(s.resultTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *placementHonorer) processRoutine(ctx context.Context, taskChan <-chan *ObjectLocationRecord) {
|
||||
loop:
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(placementHonorerEntity+ctxDoneMsg, zap.Error(ctx.Err()))
|
||||
break loop
|
||||
case locationRecord, ok := <-taskChan:
|
||||
if !ok {
|
||||
s.log.Warn(placementHonorerEntity + taskChanClosed)
|
||||
break loop
|
||||
} else if has, err := s.presenceChecker.Has(locationRecord.Address); err != nil || !has {
|
||||
continue loop
|
||||
}
|
||||
s.handleTask(ctx, locationRecord)
|
||||
}
|
||||
}
|
||||
close(s.resultChan)
|
||||
}
|
||||
|
||||
func (s *placementHonorer) handleTask(ctx context.Context, locationRecord *ObjectLocationRecord) {
|
||||
defer s.writeResult(locationRecord)
|
||||
|
||||
var (
|
||||
err error
|
||||
log = s.log.With(addressFields(locationRecord.Address)...)
|
||||
copiesShortage = locationRecord.ReservationRatio - 1
|
||||
exclNodes = make([]multiaddr.Multiaddr, 0)
|
||||
procLocations []ObjectLocation
|
||||
)
|
||||
|
||||
obj, err := s.objectSource.Get(ctx, locationRecord.Address)
|
||||
if err != nil {
|
||||
log.Warn("get object failure", zap.Error(err))
|
||||
return
|
||||
}
|
||||
|
||||
tombstone := obj.IsTombstone()
|
||||
|
||||
for copiesShortage > 0 {
|
||||
nodesInfo, err := s.remoteStorageSelector.SelectRemoteStorages(ctx, locationRecord.Address, exclNodes...)
|
||||
if err != nil {
|
||||
log.Warn("select remote storage nodes failure",
|
||||
zap.Stringer("object", locationRecord.Address),
|
||||
zap.Any("exclude nodes", exclNodes),
|
||||
zap.String("error", err.Error()),
|
||||
)
|
||||
|
||||
return
|
||||
}
|
||||
|
||||
if !tombstone {
|
||||
procLocations = make([]ObjectLocation, 0, len(nodesInfo))
|
||||
loop:
|
||||
for i := range nodesInfo {
|
||||
for j := range locationRecord.Locations {
|
||||
if locationRecord.Locations[j].Node.Equal(nodesInfo[i].Node) {
|
||||
copiesShortage--
|
||||
continue loop
|
||||
}
|
||||
}
|
||||
procLocations = append(procLocations, nodesInfo[i])
|
||||
}
|
||||
|
||||
if len(procLocations) == 0 {
|
||||
return
|
||||
}
|
||||
} else {
|
||||
procLocations = nodesInfo
|
||||
}
|
||||
|
||||
if err := s.objectReceptacle.Put(ctx, ObjectStoreParams{
|
||||
Object: obj,
|
||||
Nodes: procLocations,
|
||||
Handler: func(loc ObjectLocation, success bool) {
|
||||
if success {
|
||||
copiesShortage--
|
||||
if tombstone {
|
||||
for i := range locationRecord.Locations {
|
||||
if locationRecord.Locations[i].Node.Equal(loc.Node) {
|
||||
return
|
||||
}
|
||||
}
|
||||
}
|
||||
locationRecord.Locations = append(locationRecord.Locations, loc)
|
||||
} else {
|
||||
exclNodes = append(exclNodes, loc.Node)
|
||||
}
|
||||
},
|
||||
}); err != nil {
|
||||
s.log.Warn("put object to new nodes failure", zap.Error(err))
|
||||
return
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
// NewPlacementHonorer is a placement honorer's constructor.
|
||||
func NewPlacementHonorer(p PlacementHonorerParams) (PlacementHonorer, error) {
|
||||
switch {
|
||||
case p.RemoteStorageSelector == nil:
|
||||
return nil, instanceError(placementHonorerEntity, remoteStorageSelectorPart)
|
||||
case p.ObjectSource == nil:
|
||||
return nil, instanceError(placementHonorerEntity, objectSourcePart)
|
||||
case p.ObjectReceptacle == nil:
|
||||
return nil, instanceError(placementHonorerEntity, objectReceptaclePart)
|
||||
case p.Logger == nil:
|
||||
return nil, instanceError(placementHonorerEntity, loggerPart)
|
||||
case p.PresenceChecker == nil:
|
||||
return nil, instanceError(placementHonorerEntity, presenceCheckerPart)
|
||||
}
|
||||
|
||||
if p.TaskChanCap <= 0 {
|
||||
p.TaskChanCap = defaultPlacementHonorerChanCap
|
||||
}
|
||||
|
||||
if p.ResultTimeout <= 0 {
|
||||
p.ResultTimeout = defaultPlacementHonorerResultTimeout
|
||||
}
|
||||
|
||||
return &placementHonorer{
|
||||
objectSource: p.ObjectSource,
|
||||
objectReceptacle: p.ObjectReceptacle,
|
||||
remoteStorageSelector: p.RemoteStorageSelector,
|
||||
presenceChecker: p.PresenceChecker,
|
||||
log: p.Logger,
|
||||
taskChanCap: p.TaskChanCap,
|
||||
resultTimeout: p.ResultTimeout,
|
||||
}, nil
|
||||
}
|
194
lib/replication/storage_validator.go
Normal file
194
lib/replication/storage_validator.go
Normal file
|
@ -0,0 +1,194 @@
|
|||
package replication
|
||||
|
||||
import (
|
||||
"context"
|
||||
"time"
|
||||
|
||||
"github.com/multiformats/go-multiaddr"
|
||||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
type (
|
||||
// StorageValidator is an interface of entity
|
||||
// that listens and performs task of storage validation on remote nodes.
|
||||
// Validation can result to the need to replicate or clean object.
|
||||
StorageValidator interface {
|
||||
Process(ctx context.Context) chan<- *ObjectLocationRecord
|
||||
SubscribeReplication(ch chan<- *ReplicateTask)
|
||||
SubscribeGarbage(ch chan<- Address)
|
||||
}
|
||||
|
||||
storageValidator struct {
|
||||
objectVerifier ObjectVerifier
|
||||
log *zap.Logger
|
||||
presenceChecker PresenceChecker
|
||||
addrstore AddressStore
|
||||
|
||||
taskChanCap int
|
||||
resultTimeout time.Duration
|
||||
replicateResultChan chan<- *ReplicateTask
|
||||
garbageChan chan<- Address
|
||||
}
|
||||
|
||||
// StorageValidatorParams groups the parameters of storage validator's constructor.
|
||||
StorageValidatorParams struct {
|
||||
ObjectVerifier
|
||||
PresenceChecker
|
||||
*zap.Logger
|
||||
|
||||
TaskChanCap int
|
||||
ResultTimeout time.Duration
|
||||
AddrStore AddressStore
|
||||
}
|
||||
)
|
||||
|
||||
const (
|
||||
defaultStorageValidatorChanCap = 10
|
||||
defaultStorageValidatorResultTimeout = time.Second
|
||||
|
||||
storageValidatorEntity = "storage validator"
|
||||
)
|
||||
|
||||
func (s *storageValidator) SubscribeReplication(ch chan<- *ReplicateTask) {
|
||||
s.replicateResultChan = ch
|
||||
}
|
||||
|
||||
func (s *storageValidator) SubscribeGarbage(ch chan<- Address) { s.garbageChan = ch }
|
||||
|
||||
func (s *storageValidator) Process(ctx context.Context) chan<- *ObjectLocationRecord {
|
||||
ch := make(chan *ObjectLocationRecord, s.taskChanCap)
|
||||
go s.processRoutine(ctx, ch)
|
||||
|
||||
return ch
|
||||
}
|
||||
|
||||
func (s *storageValidator) writeReplicateResult(replicateTask *ReplicateTask) {
|
||||
if s.replicateResultChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.replicateResultChan <- replicateTask:
|
||||
case <-time.After(s.resultTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *storageValidator) writeGarbage(addr Address) {
|
||||
if s.garbageChan == nil {
|
||||
return
|
||||
}
|
||||
select {
|
||||
case s.garbageChan <- addr:
|
||||
case <-time.After(s.resultTimeout):
|
||||
s.log.Warn(writeResultTimeout)
|
||||
}
|
||||
}
|
||||
|
||||
func (s *storageValidator) processRoutine(ctx context.Context, taskChan <-chan *ObjectLocationRecord) {
|
||||
loop:
|
||||
for {
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
s.log.Warn(storageValidatorEntity+ctxDoneMsg, zap.Error(ctx.Err()))
|
||||
break loop
|
||||
case locationRecord, ok := <-taskChan:
|
||||
if !ok {
|
||||
s.log.Warn(storageValidatorEntity + taskChanClosed)
|
||||
break loop
|
||||
} else if has, err := s.presenceChecker.Has(locationRecord.Address); err != nil || !has {
|
||||
continue loop
|
||||
}
|
||||
s.handleTask(ctx, locationRecord)
|
||||
}
|
||||
}
|
||||
close(s.replicateResultChan)
|
||||
close(s.garbageChan)
|
||||
}
|
||||
|
||||
func (s *storageValidator) handleTask(ctx context.Context, locationRecord *ObjectLocationRecord) {
|
||||
selfAddr, err := s.addrstore.SelfAddr()
|
||||
if err != nil {
|
||||
s.log.Error("storage validator can't obtain self address")
|
||||
return
|
||||
}
|
||||
|
||||
var (
|
||||
weightierCounter int
|
||||
replicateTask = &ReplicateTask{
|
||||
Address: locationRecord.Address,
|
||||
Shortage: locationRecord.ReservationRatio - 1, // taking account of object correctly stored in local store
|
||||
ExcludeNodes: nodesFromLocations(locationRecord.Locations, selfAddr),
|
||||
}
|
||||
)
|
||||
|
||||
for i := range locationRecord.Locations {
|
||||
loc := locationRecord.Locations[i]
|
||||
|
||||
if s.objectVerifier.Verify(ctx, &ObjectVerificationParams{
|
||||
Address: locationRecord.Address,
|
||||
Node: locationRecord.Locations[i].Node,
|
||||
Handler: func(valid bool, _ *Object) {
|
||||
if valid {
|
||||
replicateTask.Shortage--
|
||||
if loc.WeightGreater {
|
||||
weightierCounter++
|
||||
}
|
||||
}
|
||||
},
|
||||
}); weightierCounter >= locationRecord.ReservationRatio {
|
||||
s.writeGarbage(locationRecord.Address)
|
||||
return
|
||||
}
|
||||
}
|
||||
|
||||
if replicateTask.Shortage > 0 {
|
||||
s.writeReplicateResult(replicateTask)
|
||||
}
|
||||
}
|
||||
|
||||
// nodesFromLocations must ignore self address, because it is used in
|
||||
// storage validator during replication. We must ignore our own stored
|
||||
// objects during replication and work with remote hosts and check their
|
||||
// verification info.
|
||||
func nodesFromLocations(locations []ObjectLocation, selfaddr multiaddr.Multiaddr) []multiaddr.Multiaddr {
|
||||
res := make([]multiaddr.Multiaddr, 0, len(locations))
|
||||
|
||||
for i := range locations {
|
||||
if !locations[i].Node.Equal(selfaddr) {
|
||||
res = append(res, locations[i].Node)
|
||||
}
|
||||
}
|
||||
|
||||
return res
|
||||
}
|
||||
|
||||
// NewStorageValidator is a storage validator's constructor.
|
||||
func NewStorageValidator(p StorageValidatorParams) (StorageValidator, error) {
|
||||
switch {
|
||||
case p.Logger == nil:
|
||||
return nil, instanceError(storageValidatorEntity, loggerPart)
|
||||
case p.ObjectVerifier == nil:
|
||||
return nil, instanceError(storageValidatorEntity, objectVerifierPart)
|
||||
case p.PresenceChecker == nil:
|
||||
return nil, instanceError(storageValidatorEntity, presenceCheckerPart)
|
||||
case p.AddrStore == nil:
|
||||
return nil, instanceError(storageValidatorEntity, addrStorePart)
|
||||
}
|
||||
|
||||
if p.TaskChanCap <= 0 {
|
||||
p.TaskChanCap = defaultStorageValidatorChanCap
|
||||
}
|
||||
|
||||
if p.ResultTimeout <= 0 {
|
||||
p.ResultTimeout = defaultStorageValidatorResultTimeout
|
||||
}
|
||||
|
||||
return &storageValidator{
|
||||
objectVerifier: p.ObjectVerifier,
|
||||
log: p.Logger,
|
||||
presenceChecker: p.PresenceChecker,
|
||||
taskChanCap: p.TaskChanCap,
|
||||
resultTimeout: p.ResultTimeout,
|
||||
addrstore: p.AddrStore,
|
||||
}, nil
|
||||
}
|
Loading…
Add table
Add a link
Reference in a new issue