* filer: propagate lazy metadata deletes to remote mounts Delete operations now call the remote backend for mounted remote-only entries before removing filer metadata, keeping remote state aligned and preserving retry semantics on remote failures. Made-with: Cursor * filer: harden remote delete metadata recovery Persist remote-delete metadata pendings so local entry removal can be retried after failures, and return explicit errors when remote client resolution fails to prevent silent local-only deletes. Made-with: Cursor * filer: streamline remote delete client lookup and logging Avoid a redundant mount trie traversal by resolving the remote client directly from the matched mount location, and add parity logging for successful remote directory deletions. Made-with: Cursor * filer: harden pending remote metadata deletion flow Retry pending-marker writes before local delete, fail closed when marking cannot be persisted, and start remote pending reconciliation only after the filer store is initialised to avoid nil store access. Made-with: Cursor * filer: avoid lazy fetch in pending metadata reconciliation Use a local-only entry lookup during pending remote metadata reconciliation so cache misses do not trigger remote lazy fetches. Made-with: Cursor * filer: serialise concurrent index read-modify-write in pending metadata deletion Add remoteMetadataDeletionIndexMu to Filer and acquire it for the full read→mutate→commit sequence in markRemoteMetadataDeletionPending and clearRemoteMetadataDeletionPending, preventing concurrent goroutines from overwriting each other's index updates. Made-with: Cursor * filer: start remote deletion reconciliation loop in NewFiler Move the background goroutine for pending remote metadata deletion reconciliation from SetStore (where it was gated by sync.Once) to NewFiler alongside the existing loopProcessingDeletion goroutine. The sync.Once approach was problematic: it buried a goroutine launch as a side effect of a setter, was unrecoverable if the goroutine panicked, could race with store initialisation, and coupled its lifecycle to unrelated shutdown machinery. The existing nil-store guard in reconcilePendingRemoteMetadataDeletions handles the window before SetStore is called. * filer: skip remote delete for replicated deletes from other filers When isFromOtherCluster is true the delete was already propagated to the remote backend by the originating filer. Repeating the remote delete on every replica doubles API calls, and a transient remote failure on the replica would block local metadata cleanup — leaving filers inconsistent. * filer: skip pending marking for directory remote deletes Directory remote deletes are idempotent and do not need the pending/reconcile machinery that was designed for file deletes where the local metadata delete might fail after the remote object is already removed. * filer: propagate remote deletes for children in recursive folder deletion doBatchDeleteFolderMetaAndData iterated child files but only called NotifyUpdateEvent and collected chunks — it never called maybeDeleteFromRemote for individual children. This left orphaned objects in the remote backend when a directory containing remote-only files was recursively deleted. Also fix isFromOtherCluster being hardcoded to false in the recursive call to doBatchDeleteFolderMetaAndData for subdirectories. * filer: simplify pending remote deletion tracking to single index key Replace the double-bookkeeping scheme (individual KV entry per path + newline-delimited index key) with a single index key that stores paths directly. This removes the per-path KV writes/deletes, the base64 encoding round-trip, and the transaction overhead that was only needed to keep the two representations in sync. * filer: address review feedback on remote deletion flow - Distinguish missing remote config from client initialization failure in maybeDeleteFromRemote error messages. - Use a detached context (30s timeout) for pending-mark and pending-clear KV writes so they survive request cancellation after the remote object has already been deleted. - Emit NotifyUpdateEvent in reconcilePendingRemoteMetadataDeletions after a successful retry deletion so downstream watchers and replicas learn about the eventual metadata removal. * filer: remove background reconciliation for pending remote deletions The pending-mark/reconciliation machinery (KV index, mutex, background loop, detached contexts) handled the narrow case where the remote object was deleted but the subsequent local metadata delete failed. The client already receives the error and can retry — on retry the remote not-found is treated as success and the local delete proceeds normally. The added complexity (and new edge cases around NotifyUpdateEvent, multi-filer consistency during reconciliation, and context lifetime) is not justified for a transient store failure the caller already handles. Remove: loopProcessingRemoteMetadataDeletionPending, reconcilePendingRemoteMetadataDeletions, markRemoteMetadataDeletionPending, clearRemoteMetadataDeletionPending, listPendingRemoteMetadataDeletionPaths, encodePendingRemoteMetadataDeletionIndex, FindEntryLocal, and all associated constants, fields, and test infrastructure. * filer: fix test stubs and add early exit on child remote delete error - Refactor stubFilerStore to release lock before invoking callbacks and propagate callback errors, preventing potential deadlocks in tests - Implement ListDirectoryPrefixedEntries with proper prefix filtering instead of delegating to the unfiltered ListDirectoryEntries - Add continue after setting err on child remote delete failure in doBatchDeleteFolderMetaAndData to skip further processing of the failed entry * filer: propagate child remote delete error instead of silently continuing Replace `continue` with early `break` when maybeDeleteFromRemote fails for a child entry during recursive folder deletion. The previous `continue` skipped the error check at the end of the loop body, so a subsequent successful entry would overwrite err and the remote delete error was silently lost. Now the loop breaks, the existing error check returns the error, and NotifyUpdateEvent / chunk collection are correctly skipped for the failed entry. * filer: delete remote file when entry has Remote pointer, not only when remote-only Replace IsInRemoteOnly() guard with entry.Remote == nil check in maybeDeleteFromRemote. IsInRemoteOnly() requires zero local chunks and RemoteSize > 0, which incorrectly skips remote deletion for cached files (local chunks exist) and zero-byte remote objects (RemoteSize 0). The correct condition is whether the entry has a remote backing object at all. --------- Co-authored-by: Chris Lu <chris.lu@gmail.com>
182 lines
6.0 KiB
Go
182 lines
6.0 KiB
Go
package filer
|
|
|
|
import (
|
|
"context"
|
|
"fmt"
|
|
|
|
"github.com/seaweedfs/seaweedfs/weed/glog"
|
|
"github.com/seaweedfs/seaweedfs/weed/pb/filer_pb"
|
|
"github.com/seaweedfs/seaweedfs/weed/pb/master_pb"
|
|
"github.com/seaweedfs/seaweedfs/weed/util"
|
|
)
|
|
|
|
const (
|
|
MsgFailDelNonEmptyFolder = "fail to delete non-empty folder"
|
|
)
|
|
|
|
type OnChunksFunc func([]*filer_pb.FileChunk) error
|
|
type OnHardLinkIdsFunc func([]HardLinkId) error
|
|
|
|
func (f *Filer) DeleteEntryMetaAndData(ctx context.Context, p util.FullPath, isRecursive, ignoreRecursiveError, shouldDeleteChunks, isFromOtherCluster bool, signatures []int32, ifNotModifiedAfter int64) (err error) {
|
|
if p == "/" {
|
|
return nil
|
|
}
|
|
|
|
entry, findErr := f.FindEntry(ctx, p)
|
|
if findErr != nil {
|
|
return findErr
|
|
}
|
|
if ifNotModifiedAfter > 0 && entry.Attr.Mtime.Unix() > ifNotModifiedAfter {
|
|
return nil
|
|
}
|
|
isDeleteCollection := f.IsBucket(entry)
|
|
if entry.IsDirectory() {
|
|
// delete the folder children, not including the folder itself
|
|
err = f.doBatchDeleteFolderMetaAndData(ctx, entry, isRecursive, ignoreRecursiveError, shouldDeleteChunks && !isDeleteCollection, isDeleteCollection, isFromOtherCluster, signatures, func(hardLinkIds []HardLinkId) error {
|
|
// A case not handled:
|
|
// what if the chunk is in a different collection?
|
|
if shouldDeleteChunks {
|
|
f.maybeDeleteHardLinks(ctx, hardLinkIds)
|
|
}
|
|
return nil
|
|
})
|
|
if err != nil {
|
|
glog.V(2).InfofCtx(ctx, "delete directory %s: %v", p, err)
|
|
return fmt.Errorf("delete directory %s: %v", p, err)
|
|
}
|
|
}
|
|
|
|
// delete the file or folder
|
|
err = f.doDeleteEntryMetaAndData(ctx, entry, shouldDeleteChunks, isFromOtherCluster, signatures)
|
|
if err != nil {
|
|
return fmt.Errorf("delete file %s: %v", p, err)
|
|
}
|
|
|
|
if shouldDeleteChunks && !isDeleteCollection {
|
|
if len(entry.HardLinkId) != 0 && entry.HardLinkCounter > 1 {
|
|
// if the file is a hard link and there are other hard links, do not delete the chunks
|
|
} else {
|
|
f.DeleteChunks(ctx, p, entry.GetChunks())
|
|
}
|
|
}
|
|
|
|
if isDeleteCollection {
|
|
collectionName := entry.Name()
|
|
f.DoDeleteCollection(collectionName)
|
|
}
|
|
|
|
return nil
|
|
}
|
|
|
|
func (f *Filer) doBatchDeleteFolderMetaAndData(ctx context.Context, entry *Entry, isRecursive, ignoreRecursiveError, shouldDeleteChunks, isDeletingBucket, isFromOtherCluster bool, signatures []int32, onHardLinkIdsFn OnHardLinkIdsFunc) (err error) {
|
|
|
|
//collect all the chunks of this layer and delete them together at the end
|
|
var chunksToDelete []*filer_pb.FileChunk
|
|
lastFileName := ""
|
|
includeLastFile := false
|
|
if !isDeletingBucket || !f.Store.CanDropWholeBucket() {
|
|
for {
|
|
entries, _, err := f.ListDirectoryEntries(ctx, entry.FullPath, lastFileName, includeLastFile, PaginationSize, "", "", "")
|
|
if err != nil {
|
|
glog.ErrorfCtx(ctx, "list folder %s: %v", entry.FullPath, err)
|
|
return fmt.Errorf("list folder %s: %v", entry.FullPath, err)
|
|
}
|
|
if lastFileName == "" && !isRecursive && len(entries) > 0 {
|
|
// only for first iteration in the loop
|
|
glog.V(2).InfofCtx(ctx, "deleting a folder %s has children: %+v ...", entry.FullPath, entries[0].Name())
|
|
return fmt.Errorf("%s: %s", MsgFailDelNonEmptyFolder, entry.FullPath)
|
|
}
|
|
|
|
for _, sub := range entries {
|
|
lastFileName = sub.Name()
|
|
if sub.IsDirectory() {
|
|
subIsDeletingBucket := f.IsBucket(sub)
|
|
err = f.doBatchDeleteFolderMetaAndData(ctx, sub, isRecursive, ignoreRecursiveError, shouldDeleteChunks, subIsDeletingBucket, isFromOtherCluster, nil, onHardLinkIdsFn)
|
|
} else {
|
|
if !isFromOtherCluster {
|
|
if _, remoteErr := f.maybeDeleteFromRemote(ctx, sub); remoteErr != nil {
|
|
glog.Warningf("remote delete child %s: %v", sub.FullPath, remoteErr)
|
|
if !ignoreRecursiveError {
|
|
err = remoteErr
|
|
}
|
|
}
|
|
}
|
|
if err != nil && !ignoreRecursiveError {
|
|
break
|
|
}
|
|
f.NotifyUpdateEvent(ctx, sub, nil, shouldDeleteChunks, isFromOtherCluster, nil)
|
|
if len(sub.HardLinkId) != 0 {
|
|
// hard link chunk data are deleted separately
|
|
err = onHardLinkIdsFn([]HardLinkId{sub.HardLinkId})
|
|
} else {
|
|
if shouldDeleteChunks {
|
|
chunksToDelete = append(chunksToDelete, sub.GetChunks()...)
|
|
}
|
|
}
|
|
}
|
|
if err != nil && !ignoreRecursiveError {
|
|
return err
|
|
}
|
|
}
|
|
|
|
if len(entries) < PaginationSize {
|
|
break
|
|
}
|
|
}
|
|
}
|
|
|
|
glog.V(3).InfofCtx(ctx, "deleting directory %v delete chunks: %v", entry.FullPath, shouldDeleteChunks)
|
|
|
|
if storeDeletionErr := f.Store.DeleteFolderChildren(ctx, entry.FullPath); storeDeletionErr != nil {
|
|
return fmt.Errorf("filer store delete: %w", storeDeletionErr)
|
|
}
|
|
|
|
f.NotifyUpdateEvent(ctx, entry, nil, shouldDeleteChunks, isFromOtherCluster, signatures)
|
|
f.DeleteChunks(ctx, entry.FullPath, chunksToDelete)
|
|
|
|
return nil
|
|
}
|
|
|
|
func (f *Filer) doDeleteEntryMetaAndData(ctx context.Context, entry *Entry, shouldDeleteChunks bool, isFromOtherCluster bool, signatures []int32) (err error) {
|
|
|
|
glog.V(3).InfofCtx(ctx, "deleting entry %v, delete chunks: %v", entry.FullPath, shouldDeleteChunks)
|
|
|
|
if !isFromOtherCluster {
|
|
if _, remoteDeletionErr := f.maybeDeleteFromRemote(ctx, entry); remoteDeletionErr != nil {
|
|
return remoteDeletionErr
|
|
}
|
|
}
|
|
|
|
if storeDeletionErr := f.Store.DeleteOneEntry(ctx, entry); storeDeletionErr != nil {
|
|
return fmt.Errorf("filer store delete: %w", storeDeletionErr)
|
|
}
|
|
|
|
if !entry.IsDirectory() {
|
|
f.NotifyUpdateEvent(ctx, entry, nil, shouldDeleteChunks, isFromOtherCluster, signatures)
|
|
}
|
|
|
|
return nil
|
|
}
|
|
|
|
func (f *Filer) DoDeleteCollection(collectionName string) (err error) {
|
|
|
|
return f.MasterClient.WithClient(false, func(client master_pb.SeaweedClient) error {
|
|
_, err := client.CollectionDelete(context.Background(), &master_pb.CollectionDeleteRequest{
|
|
Name: collectionName,
|
|
})
|
|
if err != nil {
|
|
glog.Infof("delete collection %s: %v", collectionName, err)
|
|
}
|
|
return err
|
|
})
|
|
|
|
}
|
|
|
|
func (f *Filer) maybeDeleteHardLinks(ctx context.Context, hardLinkIds []HardLinkId) {
|
|
for _, hardLinkId := range hardLinkIds {
|
|
if err := f.Store.DeleteHardLink(ctx, hardLinkId); err != nil {
|
|
glog.ErrorfCtx(ctx, "delete hard link id %d : %v", hardLinkId, err)
|
|
}
|
|
}
|
|
}
|