* filer.sync: send log file chunk fids to clients for direct volume server reads Instead of the server reading persisted log files from volume servers, parsing entries, and streaming them over gRPC (serial bottleneck), clients that opt in via client_supports_metadata_chunks receive log file chunk references (fids) and read directly from volume servers in parallel. New proto messages: - LogFileChunkRef: chunk fids + timestamp + filer ID for one log file - SubscribeMetadataRequest.client_supports_metadata_chunks: client opt-in - SubscribeMetadataResponse.log_file_refs: server sends refs during backlog Server changes: - CollectLogFileRefs: lists log files and returns chunk refs without any volume server I/O (metadata-only operation) - SubscribeMetadata/SubscribeLocalMetadata: when client opts in, sends refs during persisted log phase, then falls back to normal streaming for in-memory events Client changes: - ReadLogFileRefs: reads log files from volume servers, parses entries, filters by path prefix, invokes processEventFn - MetadataFollowOption.LogFileReaderFn: factory for chunk readers, enables metadata chunks when non-nil - Both filer_pb_tail.go and meta_aggregator.go recv loops accumulate refs then process them at the disk→memory transition Backward compatible: old clients don't set the flag, get existing behavior. Ref: #8771 * filer.sync: merge entries across filers in timestamp order on client side ReadLogFileRefs now groups refs by filer ID and merges entries from multiple filers using a min-heap priority queue — the same algorithm the server uses in OrderedLogVisitor + LogEntryItemPriorityQueue. This ensures events are processed in correct timestamp order even when log files from different filers have interleaved timestamps. Single-filer case takes the fast path (no heap allocation). * filer.sync: integration tests for direct-read metadata chunks Three test categories: 1. Merge correctness (TestReadLogFileRefsMergeOrder): Verifies entries from 3 filers are delivered in strict timestamp order, matching the server-side OrderedLogVisitor guarantee. 2. Path filtering (TestReadLogFileRefsPathFilter): Verifies client-side path prefix filtering works correctly. 3. Throughput comparison (TestDirectReadVsServerSideThroughput): 3 filers × 7 files × 300 events = 6300 events, 2ms per file read: server-side: 6300 events 218ms 28,873 events/sec direct-read: 6300 events 51ms 123,566 events/sec (4.3x) parallel: 6300 events 17ms 378,628 events/sec (13.1x) Direct-read eliminates gRPC send overhead per event (4.3x). Parallel per-filer reading eliminates serial file I/O (13.1x). * filer.sync: parallel per-filer reads with prefetching in ReadLogFileRefs ReadLogFileRefs now has two levels of I/O overlap: 1. Cross-filer parallelism: one goroutine per filer reads its files concurrently. Entries feed into per-filer channels, merged by the main goroutine via min-heap (same ordering guarantee as the server's OrderedLogVisitor). 2. Within-filer prefetching: while the current file's entries are being consumed by the merge heap, the next file is already being read from the volume server in a background goroutine. Single-filer fast path avoids the heap and channels. Test results (3 filers × 7 files × 300 events, 2ms per file read): server-side sequential: 6300 events 212ms 29,760 events/sec parallel + prefetch: 6300 events 36ms 177,443 events/sec Speedup: 6.0x * filer.sync: address all review comments on metadata chunks PR Critical fixes: - sendLogFileRefs: bypass pipelinedSender, send directly on gRPC stream. Ref messages have TsNs=0 and were being incorrectly batched into the Events field by the adaptive batching logic, corrupting ref delivery. - readLogFileEntries: use io.ReadFull instead of reader.Read to prevent partial reads from corrupting size values or protobuf data. - Error handling: only skip chunk-not-found errors (matching server-side isChunkNotFoundError). Other I/O or decode failures are propagated so the follower can retry. High-priority fixes: - CollectLogFileRefs: remove incorrect +24h padding from stopTime. The extra day caused unnecessary log file refs to be collected. - Path filtering: ReadLogFileRefs now accepts PathFilter struct with PathPrefix, AdditionalPathPrefixes, and DirectoriesToWatch. Uses util.Join for path construction (avoids "//foo" on root). Excludes /.system/log/ internal entries. Matches server-side eachEventNotificationFn filtering logic. Medium-priority fixes: - CollectLogFileRefs: accept context.Context, propagate to ListDirectoryEntries calls for cancellation support. - NewChunkStreamReaderFromLookup: accept context.Context, propagate to doNewChunkStreamReader. Test fixes: - Check error returns from ReadLogFileRefs in all test call sites. --------- Co-authored-by: Copilot <copilot@github.com>
357 lines
11 KiB
Go
357 lines
11 KiB
Go
package filer
|
|
|
|
import (
|
|
"context"
|
|
"fmt"
|
|
"io"
|
|
"strings"
|
|
"sync"
|
|
"sync/atomic"
|
|
"time"
|
|
|
|
"github.com/seaweedfs/seaweedfs/weed/pb/master_pb"
|
|
"github.com/seaweedfs/seaweedfs/weed/util"
|
|
|
|
"google.golang.org/grpc"
|
|
"google.golang.org/protobuf/proto"
|
|
|
|
"github.com/seaweedfs/seaweedfs/weed/glog"
|
|
"github.com/seaweedfs/seaweedfs/weed/pb"
|
|
"github.com/seaweedfs/seaweedfs/weed/pb/filer_pb"
|
|
"github.com/seaweedfs/seaweedfs/weed/util/log_buffer"
|
|
)
|
|
|
|
type MetaAggregator struct {
|
|
filer *Filer
|
|
self pb.ServerAddress
|
|
isLeader bool
|
|
grpcDialOption grpc.DialOption
|
|
MetaLogBuffer *log_buffer.LogBuffer
|
|
peerChans map[pb.ServerAddress]chan struct{}
|
|
peerChansLock sync.Mutex
|
|
// notifying clients
|
|
ListenersLock sync.Mutex
|
|
ListenersWaits int64 // Atomic counter
|
|
ListenersCond *sync.Cond
|
|
}
|
|
|
|
// MetaAggregator only aggregates data "on the fly". The logs are not re-persisted to disk.
|
|
// The old data comes from what each LocalMetadata persisted on disk.
|
|
func NewMetaAggregator(filer *Filer, self pb.ServerAddress, grpcDialOption grpc.DialOption) *MetaAggregator {
|
|
t := &MetaAggregator{
|
|
filer: filer,
|
|
self: self,
|
|
grpcDialOption: grpcDialOption,
|
|
peerChans: make(map[pb.ServerAddress]chan struct{}),
|
|
}
|
|
t.ListenersCond = sync.NewCond(&t.ListenersLock)
|
|
t.MetaLogBuffer = log_buffer.NewLogBuffer("aggr", LogFlushInterval, nil, nil, func() {
|
|
if atomic.LoadInt64(&t.ListenersWaits) > 0 {
|
|
t.ListenersCond.Broadcast()
|
|
}
|
|
})
|
|
return t
|
|
}
|
|
|
|
func (ma *MetaAggregator) OnPeerUpdate(update *master_pb.ClusterNodeUpdate, startFrom time.Time) {
|
|
ma.peerChansLock.Lock()
|
|
defer ma.peerChansLock.Unlock()
|
|
|
|
address := pb.ServerAddress(update.Address)
|
|
if update.IsAdd {
|
|
// cancel previous subscription if any
|
|
if prevChan, found := ma.peerChans[address]; found {
|
|
close(prevChan)
|
|
}
|
|
stopChan := make(chan struct{})
|
|
ma.peerChans[address] = stopChan
|
|
go ma.loopSubscribeToOneFiler(ma.filer, ma.self, address, startFrom, stopChan)
|
|
} else {
|
|
if prevChan, found := ma.peerChans[address]; found {
|
|
close(prevChan)
|
|
delete(ma.peerChans, address)
|
|
}
|
|
}
|
|
}
|
|
|
|
func (ma *MetaAggregator) HasRemotePeers() bool {
|
|
ma.peerChansLock.Lock()
|
|
defer ma.peerChansLock.Unlock()
|
|
|
|
for address := range ma.peerChans {
|
|
if address != ma.self {
|
|
return true
|
|
}
|
|
}
|
|
return false
|
|
}
|
|
|
|
func (ma *MetaAggregator) loopSubscribeToOneFiler(f *Filer, self pb.ServerAddress, peer pb.ServerAddress, startFrom time.Time, stopChan chan struct{}) {
|
|
lastTsNs := startFrom.UnixNano()
|
|
for {
|
|
glog.V(0).Infof("loopSubscribeToOneFiler read %s start from %v %d", peer, time.Unix(0, lastTsNs), lastTsNs)
|
|
nextLastTsNs, err := ma.doSubscribeToOneFiler(f, self, peer, lastTsNs)
|
|
|
|
// check stopChan to see if we should stop
|
|
select {
|
|
case <-stopChan:
|
|
glog.V(0).Infof("stop subscribing peer %s meta change", peer)
|
|
return
|
|
default:
|
|
}
|
|
|
|
if err != nil {
|
|
errLvl := glog.Level(0)
|
|
if strings.Contains(err.Error(), "duplicated local subscription detected") {
|
|
errLvl = glog.Level(4)
|
|
}
|
|
glog.V(errLvl).Infof("subscribing remote %s meta change: %v", peer, err)
|
|
}
|
|
if lastTsNs < nextLastTsNs {
|
|
lastTsNs = nextLastTsNs
|
|
}
|
|
time.Sleep(1733 * time.Millisecond)
|
|
}
|
|
}
|
|
|
|
func (ma *MetaAggregator) doSubscribeToOneFiler(f *Filer, self pb.ServerAddress, peer pb.ServerAddress, startFrom int64) (int64, error) {
|
|
|
|
/*
|
|
Each filer reads the "filer.store.id", which is the store's signature when filer starts.
|
|
|
|
When reading from other filers' local meta changes:
|
|
* if the received change does not contain signature from self, apply the change to current filer store.
|
|
|
|
Upon connecting to other filers, need to remember their signature and their offsets.
|
|
|
|
*/
|
|
|
|
var maybeReplicateMetadataChange func(*filer_pb.SubscribeMetadataResponse)
|
|
lastPersistTime := time.Now()
|
|
lastTsNs := startFrom
|
|
|
|
peerSignature, err := ma.readFilerStoreSignature(peer)
|
|
if err != nil {
|
|
return lastTsNs, fmt.Errorf("connecting to peer filer %s: %v", peer, err)
|
|
}
|
|
|
|
// when filer store is not shared by multiple filers
|
|
if peerSignature != f.Signature {
|
|
if prevTsNs, err := ma.readOffset(f, peer, peerSignature); err == nil {
|
|
lastTsNs = prevTsNs
|
|
defer func(prevTsNs int64) {
|
|
if lastTsNs != prevTsNs && lastTsNs != lastPersistTime.UnixNano() {
|
|
if err := ma.updateOffset(f, peer, peerSignature, lastTsNs); err == nil {
|
|
glog.V(0).Infof("last sync time with %s at %v (%d)", peer, time.Unix(0, lastTsNs), lastTsNs)
|
|
} else {
|
|
glog.Errorf("failed to save last sync time with %s at %v (%d)", peer, time.Unix(0, lastTsNs), lastTsNs)
|
|
}
|
|
}
|
|
}(prevTsNs)
|
|
}
|
|
|
|
glog.V(0).Infof("follow peer: %v, last %v (%d)", peer, time.Unix(0, lastTsNs), lastTsNs)
|
|
var counter int64
|
|
var synced bool
|
|
maybeReplicateMetadataChange = func(event *filer_pb.SubscribeMetadataResponse) {
|
|
if err := Replay(f.Store, event); err != nil {
|
|
glog.Errorf("failed to reply metadata change from %v: %v", peer, err)
|
|
return
|
|
}
|
|
counter++
|
|
if lastPersistTime.Add(time.Minute).Before(time.Now()) {
|
|
if err := ma.updateOffset(f, peer, peerSignature, event.TsNs); err == nil {
|
|
if event.TsNs < time.Now().Add(-2*time.Minute).UnixNano() {
|
|
glog.V(0).Infof("sync with %s progressed to: %v %0.2f/sec", peer, time.Unix(0, event.TsNs), float64(counter)/60.0)
|
|
} else if !synced {
|
|
synced = true
|
|
glog.V(0).Infof("synced with %s", peer)
|
|
}
|
|
lastPersistTime = time.Now()
|
|
counter = 0
|
|
} else {
|
|
glog.V(0).Infof("failed to update offset for %v: %v", peer, err)
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
processEventFn := func(event *filer_pb.SubscribeMetadataResponse) error {
|
|
data, err := proto.Marshal(event)
|
|
if err != nil {
|
|
glog.Errorf("failed to marshal subscribed filer_pb.SubscribeMetadataResponse %+v: %v", event, err)
|
|
return err
|
|
}
|
|
dir := event.Directory
|
|
// println("received meta change", dir, "size", len(data))
|
|
if err := ma.MetaLogBuffer.AddDataToBuffer([]byte(dir), data, event.TsNs); err != nil {
|
|
glog.Errorf("failed to add data to log buffer for %s: %v", dir, err)
|
|
return err
|
|
}
|
|
if maybeReplicateMetadataChange != nil {
|
|
maybeReplicateMetadataChange(event)
|
|
}
|
|
return nil
|
|
}
|
|
|
|
glog.V(0).Infof("subscribing remote %s meta change: %v, clientId:%d", peer, time.Unix(0, lastTsNs), ma.filer.UniqueFilerId)
|
|
err = pb.WithFilerClient(true, 0, peer, ma.grpcDialOption, func(client filer_pb.SeaweedFilerClient) error {
|
|
ctx, cancel := context.WithCancel(context.Background())
|
|
defer cancel()
|
|
atomic.AddInt32(&ma.filer.UniqueFilerEpoch, 1)
|
|
// Construct a log file reader that reads chunks via the peer filer's LookupVolume.
|
|
lookupFn := LookupFn(filerClient{client})
|
|
logFileReaderFn := func(chunks []*filer_pb.FileChunk) (io.ReadCloser, error) {
|
|
return NewChunkStreamReaderFromLookup(ctx, lookupFn, chunks), nil
|
|
}
|
|
|
|
stream, err := client.SubscribeLocalMetadata(ctx, &filer_pb.SubscribeMetadataRequest{
|
|
ClientName: "filer:" + string(self),
|
|
PathPrefix: "/",
|
|
SinceNs: lastTsNs,
|
|
ClientId: ma.filer.UniqueFilerId,
|
|
ClientEpoch: atomic.LoadInt32(&ma.filer.UniqueFilerEpoch),
|
|
ClientSupportsBatching: true,
|
|
ClientSupportsMetadataChunks: true,
|
|
})
|
|
if err != nil {
|
|
glog.V(0).Infof("SubscribeLocalMetadata %v: %v", peer, err)
|
|
return fmt.Errorf("subscribe: %w", err)
|
|
}
|
|
|
|
processOne := func(event *filer_pb.SubscribeMetadataResponse) error {
|
|
if err := processEventFn(event); err != nil {
|
|
glog.V(0).Infof("SubscribeLocalMetadata process %v: %v", event, err)
|
|
return fmt.Errorf("process %v: %w", event, err)
|
|
}
|
|
f.onMetadataChangeEvent(event)
|
|
lastTsNs = event.TsNs
|
|
return nil
|
|
}
|
|
|
|
var pendingRefs []*filer_pb.LogFileChunkRef
|
|
|
|
for {
|
|
resp, listenErr := stream.Recv()
|
|
if listenErr == io.EOF {
|
|
return nil
|
|
}
|
|
if listenErr != nil {
|
|
glog.V(0).Infof("SubscribeLocalMetadata stream %v: %v", peer, listenErr)
|
|
return listenErr
|
|
}
|
|
|
|
// Accumulate log file chunk references
|
|
if len(resp.LogFileRefs) > 0 {
|
|
pendingRefs = append(pendingRefs, resp.LogFileRefs...)
|
|
continue
|
|
}
|
|
|
|
// Process accumulated refs (transition from disk to in-memory)
|
|
if len(pendingRefs) > 0 {
|
|
lastTs, readErr := pb.ReadLogFileRefs(pendingRefs, logFileReaderFn,
|
|
lastTsNs, 0, pb.PathFilter{PathPrefix: "/"},
|
|
func(event *filer_pb.SubscribeMetadataResponse) error {
|
|
return processOne(event)
|
|
})
|
|
if readErr != nil {
|
|
return fmt.Errorf("read log file refs from %s: %w", peer, readErr)
|
|
}
|
|
if lastTs > 0 {
|
|
lastTsNs = lastTs
|
|
}
|
|
pendingRefs = nil
|
|
}
|
|
|
|
if resp.EventNotification != nil {
|
|
if err := processOne(resp); err != nil {
|
|
return err
|
|
}
|
|
}
|
|
// Process any additional batched events
|
|
for _, batchedEvent := range resp.Events {
|
|
if err := processOne(batchedEvent); err != nil {
|
|
return err
|
|
}
|
|
}
|
|
}
|
|
})
|
|
return lastTsNs, err
|
|
}
|
|
|
|
func (ma *MetaAggregator) readFilerStoreSignature(peer pb.ServerAddress) (sig int32, err error) {
|
|
err = pb.WithFilerClient(false, 0, peer, ma.grpcDialOption, func(client filer_pb.SeaweedFilerClient) error {
|
|
resp, err := client.GetFilerConfiguration(context.Background(), &filer_pb.GetFilerConfigurationRequest{})
|
|
if err != nil {
|
|
return err
|
|
}
|
|
sig = resp.Signature
|
|
return nil
|
|
})
|
|
return
|
|
}
|
|
|
|
const (
|
|
MetaOffsetPrefix = "Meta"
|
|
)
|
|
|
|
func GetPeerMetaOffsetKey(peerSignature int32) []byte {
|
|
key := []byte(MetaOffsetPrefix + "xxxx")
|
|
util.Uint32toBytes(key[len(MetaOffsetPrefix):], uint32(peerSignature))
|
|
return key
|
|
}
|
|
|
|
func (ma *MetaAggregator) readOffset(f *Filer, peer pb.ServerAddress, peerSignature int32) (lastTsNs int64, err error) {
|
|
|
|
key := GetPeerMetaOffsetKey(peerSignature)
|
|
|
|
value, err := f.Store.KvGet(context.Background(), key)
|
|
|
|
if err != nil {
|
|
return 0, fmt.Errorf("readOffset %s : %v", peer, err)
|
|
}
|
|
|
|
lastTsNs = int64(util.BytesToUint64(value))
|
|
|
|
glog.V(0).Infof("readOffset %s : %d", peer, lastTsNs)
|
|
|
|
return
|
|
}
|
|
|
|
func (ma *MetaAggregator) updateOffset(f *Filer, peer pb.ServerAddress, peerSignature int32, lastTsNs int64) (err error) {
|
|
|
|
key := GetPeerMetaOffsetKey(peerSignature)
|
|
|
|
value := make([]byte, 8)
|
|
util.Uint64toBytes(value, uint64(lastTsNs))
|
|
|
|
err = f.Store.KvPut(context.Background(), key, value)
|
|
|
|
if err != nil {
|
|
return fmt.Errorf("updateOffset %s : %v", peer, err)
|
|
}
|
|
|
|
glog.V(4).Infof("updateOffset %s : %d", peer, lastTsNs)
|
|
|
|
return
|
|
}
|
|
|
|
// filerClient adapts a SeaweedFilerClient to the FilerClient interface
|
|
// for use with LookupFn. Used by MetaAggregator to resolve volume IDs
|
|
// on peer filers.
|
|
type filerClient struct {
|
|
client filer_pb.SeaweedFilerClient
|
|
}
|
|
|
|
func (fc filerClient) WithFilerClient(streamingMode bool, fn func(filer_pb.SeaweedFilerClient) error) error {
|
|
return fn(fc.client)
|
|
}
|
|
|
|
func (fc filerClient) AdjustedUrl(location *filer_pb.Location) string {
|
|
return location.Url
|
|
}
|
|
|
|
func (fc filerClient) GetDataCenter() string {
|
|
return ""
|
|
}
|