Files
seaweedFS/weed/filer/filer_notify_read.go
Chris Lu c2c58419b8 filer.sync: send log file chunk fids to clients for direct volume server reads (#8792)
* 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>
2026-03-27 11:01:29 -07:00

485 lines
14 KiB
Go

package filer
import (
"container/heap"
"context"
"fmt"
"io"
"math"
"strings"
"time"
"github.com/seaweedfs/seaweedfs/weed/pb/filer_pb"
"github.com/seaweedfs/seaweedfs/weed/util/log_buffer"
"github.com/seaweedfs/seaweedfs/weed/wdclient"
"google.golang.org/protobuf/proto"
"github.com/seaweedfs/seaweedfs/weed/glog"
"github.com/seaweedfs/seaweedfs/weed/util"
)
type LogFileEntry struct {
TsNs int64
FileEntry *Entry
}
func (f *Filer) collectPersistedLogBuffer(startPosition log_buffer.MessagePosition, stopTsNs int64) (v *OrderedLogVisitor, err error) {
if stopTsNs != 0 && startPosition.Time.UnixNano() > stopTsNs {
return nil, io.EOF
}
startDate := fmt.Sprintf("%04d-%02d-%02d", startPosition.Time.Year(), startPosition.Time.Month(), startPosition.Time.Day())
dayEntries, _, listDayErr := f.ListDirectoryEntries(context.Background(), SystemLogDir, startDate, true, math.MaxInt32, "", "", "")
if listDayErr != nil {
return nil, fmt.Errorf("fail to list log by day: %w", listDayErr)
}
return NewOrderedLogVisitor(f, startPosition, stopTsNs, dayEntries)
}
// CollectLogFileRefs lists persisted log files and returns their chunk references
// without reading any data from volume servers. The client can use the returned
// fids to read log file data directly from volume servers in parallel.
func (f *Filer) CollectLogFileRefs(ctx context.Context, startPosition log_buffer.MessagePosition, stopTsNs int64) (refs []*filer_pb.LogFileChunkRef, lastTsNs int64, err error) {
if stopTsNs != 0 && startPosition.Time.UnixNano() > stopTsNs {
return nil, 0, nil
}
startDate := fmt.Sprintf("%04d-%02d-%02d", startPosition.Time.Year(), startPosition.Time.Month(), startPosition.Time.Day())
startHourMinute := fmt.Sprintf("%02d-%02d", startPosition.Time.Hour(), startPosition.Time.Minute())
var stopDate, stopHourMinute string
if stopTsNs != 0 {
stopTime := time.Unix(0, stopTsNs).UTC()
stopDate = fmt.Sprintf("%04d-%02d-%02d", stopTime.Year(), stopTime.Month(), stopTime.Day())
stopHourMinute = fmt.Sprintf("%02d-%02d", stopTime.Hour(), stopTime.Minute())
}
dayEntries, _, listDayErr := f.ListDirectoryEntries(ctx, SystemLogDir, startDate, true, math.MaxInt32, "", "", "")
if listDayErr != nil {
return nil, 0, fmt.Errorf("fail to list log by day: %w", listDayErr)
}
for _, dayEntry := range dayEntries {
if stopDate != "" && strings.Compare(dayEntry.Name(), stopDate) > 0 {
break
}
hourMinuteEntries, _, listErr := f.ListDirectoryEntries(ctx, util.NewFullPath(SystemLogDir, dayEntry.Name()), "", false, math.MaxInt32, "", "", "")
if listErr != nil {
return nil, 0, fmt.Errorf("fail to list log %s: %w", dayEntry.Name(), listErr)
}
for _, hmEntry := range hourMinuteEntries {
hourMinute := util.FileNameBase(hmEntry.Name())
if dayEntry.Name() == startDate && strings.Compare(hourMinute, startHourMinute) < 0 {
continue
}
if dayEntry.Name() == stopDate && stopHourMinute != "" && strings.Compare(hourMinute, stopHourMinute) > 0 {
break
}
tsMinute := fmt.Sprintf("%s-%s", dayEntry.Name(), hourMinute)
t, parseErr := time.Parse("2006-01-02-15-04", tsMinute)
if parseErr != nil {
glog.Errorf("failed to parse %s: %v", tsMinute, parseErr)
continue
}
filerId := getFilerId(hmEntry.Name())
if filerId == "" {
continue
}
chunks := hmEntry.GetChunks()
if len(chunks) == 0 {
continue
}
refs = append(refs, &filer_pb.LogFileChunkRef{
Chunks: chunks,
FileTsNs: t.UnixNano(),
FilerId: filerId,
})
lastTsNs = t.UnixNano()
}
}
return
}
func (f *Filer) HasPersistedLogFiles(startPosition log_buffer.MessagePosition) (bool, error) {
startDate := fmt.Sprintf("%04d-%02d-%02d", startPosition.Time.Year(), startPosition.Time.Month(), startPosition.Time.Day())
dayEntries, _, listDayErr := f.ListDirectoryEntries(context.Background(), SystemLogDir, startDate, true, 1, "", "", "")
if listDayErr != nil {
return false, fmt.Errorf("fail to list log by day: %w", listDayErr)
}
if len(dayEntries) == 0 {
return false, nil
}
return true, nil
}
// ----------
type LogEntryItem struct {
Entry *filer_pb.LogEntry
filer string
}
// LogEntryItemPriorityQueue a priority queue for LogEntry
type LogEntryItemPriorityQueue []*LogEntryItem
func (pq LogEntryItemPriorityQueue) Len() int { return len(pq) }
func (pq LogEntryItemPriorityQueue) Less(i, j int) bool {
return pq[i].Entry.TsNs < pq[j].Entry.TsNs
}
func (pq LogEntryItemPriorityQueue) Swap(i, j int) { pq[i], pq[j] = pq[j], pq[i] }
func (pq *LogEntryItemPriorityQueue) Push(x any) {
item := x.(*LogEntryItem)
*pq = append(*pq, item)
}
func (pq *LogEntryItemPriorityQueue) Pop() any {
n := len(*pq)
item := (*pq)[n-1]
*pq = (*pq)[:n-1]
return item
}
// ----------
type OrderedLogVisitor struct {
perFilerIteratorMap map[string]*LogFileQueueIterator
pq *LogEntryItemPriorityQueue
logFileEntryCollector *LogFileEntryCollector
}
func NewOrderedLogVisitor(f *Filer, startPosition log_buffer.MessagePosition, stopTsNs int64, dayEntries []*Entry) (*OrderedLogVisitor, error) {
perFilerQueueMap := make(map[string]*LogFileQueueIterator)
// initialize the priority queue
pq := &LogEntryItemPriorityQueue{}
heap.Init(pq)
t := &OrderedLogVisitor{
perFilerIteratorMap: perFilerQueueMap,
pq: pq,
logFileEntryCollector: NewLogFileEntryCollector(f, startPosition, stopTsNs, dayEntries),
}
if err := t.logFileEntryCollector.collectMore(t); err != nil && err != io.EOF {
return nil, err
}
return t, nil
}
func (o *OrderedLogVisitor) GetNext() (logEntry *filer_pb.LogEntry, err error) {
if o.pq.Len() == 0 {
return nil, io.EOF
}
item := heap.Pop(o.pq).(*LogEntryItem)
filerId := item.filer
// fill the pq with the next log entry from the same filer
it := o.perFilerIteratorMap[filerId]
next, nextErr := it.getNext(o)
if nextErr != nil {
if nextErr == io.EOF {
// do nothing since the filer has no more log entries
} else {
return nil, fmt.Errorf("failed to get next log entry: %w", nextErr)
}
} else {
heap.Push(o.pq, &LogEntryItem{
Entry: next,
filer: filerId,
})
}
return item.Entry, nil
}
func getFilerId(name string) string {
idx := strings.LastIndex(name, ".")
if idx < 0 {
return ""
}
return name[idx+1:]
}
// ----------
type LogFileEntryCollector struct {
f *Filer
startTsNs int64
stopTsNs int64
dayEntryQueue *util.Queue[*Entry]
startDate string
startHourMinute string
stopDate string
stopHourMinute string
}
func NewLogFileEntryCollector(f *Filer, startPosition log_buffer.MessagePosition, stopTsNs int64, dayEntries []*Entry) *LogFileEntryCollector {
dayEntryQueue := util.NewQueue[*Entry]()
for _, dayEntry := range dayEntries {
dayEntryQueue.Enqueue(dayEntry)
// println("enqueue day entry", dayEntry.Name())
}
startDate := fmt.Sprintf("%04d-%02d-%02d", startPosition.Time.Year(), startPosition.Time.Month(), startPosition.Time.Day())
startHourMinute := fmt.Sprintf("%02d-%02d", startPosition.Time.Hour(), startPosition.Time.Minute())
var stopDate, stopHourMinute string
if stopTsNs != 0 {
stopTime := time.Unix(0, stopTsNs+24*60*60*int64(time.Second)).UTC()
stopDate = fmt.Sprintf("%04d-%02d-%02d", stopTime.Year(), stopTime.Month(), stopTime.Day())
stopHourMinute = fmt.Sprintf("%02d-%02d", stopTime.Hour(), stopTime.Minute())
}
return &LogFileEntryCollector{
f: f,
startTsNs: startPosition.Time.UnixNano(),
stopTsNs: stopTsNs,
dayEntryQueue: dayEntryQueue,
startDate: startDate,
startHourMinute: startHourMinute,
stopDate: stopDate,
stopHourMinute: stopHourMinute,
}
}
func (c *LogFileEntryCollector) hasMore() bool {
return c.dayEntryQueue.Len() > 0
}
func (c *LogFileEntryCollector) collectMore(v *OrderedLogVisitor) (err error) {
dayEntry := c.dayEntryQueue.Dequeue()
if dayEntry == nil {
return io.EOF
}
// println("dequeue day entry", dayEntry.Name())
if c.stopDate != "" {
if strings.Compare(dayEntry.Name(), c.stopDate) > 0 {
return io.EOF
}
}
hourMinuteEntries, _, listHourMinuteErr := c.f.ListDirectoryEntries(context.Background(), util.NewFullPath(SystemLogDir, dayEntry.Name()), "", false, math.MaxInt32, "", "", "")
if listHourMinuteErr != nil {
return fmt.Errorf("fail to list log %s by day: %v", dayEntry.Name(), listHourMinuteErr)
}
freshFilerIds := make(map[string]string)
for _, hourMinuteEntry := range hourMinuteEntries {
// println("checking hh-mm", hourMinuteEntry.FullPath)
hourMinute := util.FileNameBase(hourMinuteEntry.Name())
if dayEntry.Name() == c.startDate {
if strings.Compare(hourMinute, c.startHourMinute) < 0 {
continue
}
}
if dayEntry.Name() == c.stopDate {
if strings.Compare(hourMinute, c.stopHourMinute) > 0 {
break
}
}
tsMinute := fmt.Sprintf("%s-%s", dayEntry.Name(), hourMinute)
// println(" enqueue", tsMinute)
t, parseErr := time.Parse("2006-01-02-15-04", tsMinute)
if parseErr != nil {
glog.Errorf("failed to parse %s: %v", tsMinute, parseErr)
continue
}
filerId := getFilerId(hourMinuteEntry.Name())
if filerId == "" {
glog.Warningf("Invalid log file name format: %s", hourMinuteEntry.Name())
continue // Skip files with invalid format
}
iter, found := v.perFilerIteratorMap[filerId]
if !found {
iter = newLogFileQueueIterator(c.f.MasterClient, util.NewQueue[*LogFileEntry](), c.startTsNs, c.stopTsNs)
v.perFilerIteratorMap[filerId] = iter
freshFilerIds[filerId] = hourMinuteEntry.Name()
}
iter.q.Enqueue(&LogFileEntry{
TsNs: t.UnixNano(),
FileEntry: hourMinuteEntry,
})
}
// fill the pq with the next log entry if it is a new filer
for filerId, entryName := range freshFilerIds {
iter, found := v.perFilerIteratorMap[filerId]
if !found {
glog.Errorf("Unexpected! failed to find iterator for filer %s", filerId)
continue
}
next, nextErr := iter.getNext(v)
if nextErr != nil {
if nextErr == io.EOF {
// do nothing since the filer has no more log entries
} else {
return fmt.Errorf("failed to get next log entry for %v: %w", entryName, nextErr)
}
} else {
heap.Push(v.pq, &LogEntryItem{
Entry: next,
filer: filerId,
})
}
}
return nil
}
// ----------
type LogFileQueueIterator struct {
q *util.Queue[*LogFileEntry]
masterClient *wdclient.MasterClient
startTsNs int64
stopTsNs int64
pendingEntries []*filer_pb.LogEntry
pendingIndex int
}
func newLogFileQueueIterator(masterClient *wdclient.MasterClient, q *util.Queue[*LogFileEntry], startTsNs, stopTsNs int64) *LogFileQueueIterator {
return &LogFileQueueIterator{
q: q,
masterClient: masterClient,
startTsNs: startTsNs,
stopTsNs: stopTsNs,
}
}
// getNext will return io.EOF when done
func (iter *LogFileQueueIterator) getNext(v *OrderedLogVisitor) (logEntry *filer_pb.LogEntry, err error) {
for {
// return pending entries first
if iter.pendingIndex < len(iter.pendingEntries) {
logEntry = iter.pendingEntries[iter.pendingIndex]
iter.pendingIndex++
return logEntry, nil
}
// reset for next file
iter.pendingEntries = nil
iter.pendingIndex = 0
// read entries from next file
if iter.q.Len() == 0 {
return nil, io.EOF
}
t := iter.q.Dequeue()
if t == nil {
continue
}
// skip the file if it is after the stopTsNs
if iter.stopTsNs != 0 && t.TsNs > iter.stopTsNs {
return nil, io.EOF
}
next := iter.q.Peek()
if next == nil {
if collectErr := v.logFileEntryCollector.collectMore(v); collectErr != nil && collectErr != io.EOF {
return nil, collectErr
}
next = iter.q.Peek() // Re-peek after collectMore
}
// skip the file if the next entry is before the startTsNs
if next != nil && next.TsNs <= iter.startTsNs {
continue
}
// read all entries from this file
iter.pendingEntries, err = iter.readFileEntries(t.FileEntry)
if err != nil {
return nil, err
}
}
}
// readFileEntries reads all log entries from a single file
func (iter *LogFileQueueIterator) readFileEntries(fileEntry *Entry) (entries []*filer_pb.LogEntry, err error) {
fileIterator := newLogFileIterator(iter.masterClient, fileEntry, iter.startTsNs, iter.stopTsNs)
defer func() {
if closeErr := fileIterator.Close(); closeErr != nil && err == nil {
err = closeErr
}
}()
for {
logEntry, err := fileIterator.getNext()
if err == io.EOF {
return entries, nil
}
if err != nil {
if isChunkNotFoundError(err) {
// Volume or chunk was deleted, skip the rest of this log file
glog.Warningf("skipping rest of %s: %v", fileIterator.filePath, err)
return entries, nil
}
return nil, err
}
entries = append(entries, logEntry)
}
}
// ----------
type LogFileIterator struct {
r io.Reader
sizeBuf []byte
startTsNs int64
stopTsNs int64
filePath string
}
func newLogFileIterator(masterClient *wdclient.MasterClient, fileEntry *Entry, startTsNs, stopTsNs int64) *LogFileIterator {
return &LogFileIterator{
r: NewChunkStreamReaderFromFiler(context.Background(), masterClient, fileEntry.Chunks),
sizeBuf: make([]byte, 4),
startTsNs: startTsNs,
stopTsNs: stopTsNs,
filePath: string(fileEntry.FullPath),
}
}
func (iter *LogFileIterator) Close() error {
if r, ok := iter.r.(io.Closer); ok {
return r.Close()
}
return nil
}
// getNext will return io.EOF when done
func (iter *LogFileIterator) getNext() (logEntry *filer_pb.LogEntry, err error) {
var n int
for {
n, err = iter.r.Read(iter.sizeBuf)
if err != nil {
return
}
if n != 4 {
return nil, fmt.Errorf("size %d bytes, expected 4 bytes", n)
}
size := util.BytesToUint32(iter.sizeBuf)
// println("entry size", size)
entryData := make([]byte, size)
n, err = iter.r.Read(entryData)
if err != nil {
return
}
if n != int(size) {
return nil, fmt.Errorf("entry data %d bytes, expected %d bytes", n, size)
}
logEntry = &filer_pb.LogEntry{}
if err = proto.Unmarshal(entryData, logEntry); err != nil {
return
}
if logEntry.TsNs <= iter.startTsNs {
continue
}
if iter.stopTsNs != 0 && logEntry.TsNs > iter.stopTsNs {
return nil, io.EOF
}
return
}
}