Admin UI: Fetch task logs (#7114)

* show task details

* loading tasks

* task UI works

* generic rendering

* rendering the export link

* removing placementConflicts from task parameters

* remove TaskSourceLocation

* remove "Server ID" column

* rendering balance task source

* sources and targets

* fix ec task generation

* move info

* render timeline

* simplified worker id

* simplify

* read task logs from worker

* isValidTaskID

* address comments

* Update weed/worker/tasks/balance/execution.go

Co-authored-by: Copilot <175728472+Copilot@users.noreply.github.com>

* Update weed/worker/tasks/erasure_coding/ec_task.go

Co-authored-by: Copilot <175728472+Copilot@users.noreply.github.com>

* Update weed/worker/tasks/task_log_handler.go

Co-authored-by: Copilot <175728472+Copilot@users.noreply.github.com>

* fix shard ids

* plan distributing shard id

* rendering planned shards in task details

* remove Conflicts

* worker logs correctly

* pass in dc and rack

* task logging

* Update weed/admin/maintenance/maintenance_queue.go

Co-authored-by: gemini-code-assist[bot] <176961590+gemini-code-assist[bot]@users.noreply.github.com>

* display log details

* logs have fields now

* sort field keys

* fix link

* fix collection filtering

* avoid hard coded ec shard counts

---------

Co-authored-by: Copilot <175728472+Copilot@users.noreply.github.com>
Co-authored-by: gemini-code-assist[bot] <176961590+gemini-code-assist[bot]@users.noreply.github.com>
This commit is contained in:
Chris Lu
2025-08-09 21:47:29 -07:00
committed by GitHub
parent 3ac2a2e22d
commit 25bbf4c3d4
52 changed files with 7307 additions and 2004 deletions

View File

@@ -651,7 +651,7 @@ func (c *GrpcAdminClient) RequestTask(workerID string, capabilities []types.Task
Type: types.TaskType(taskAssign.TaskType),
Status: types.TaskStatusAssigned,
VolumeID: taskAssign.Params.VolumeId,
Server: taskAssign.Params.Server,
Server: getServerFromParams(taskAssign.Params),
Collection: taskAssign.Params.Collection,
Priority: types.TaskPriority(taskAssign.Priority),
CreatedAt: time.Unix(taskAssign.CreatedTime, 0),
@@ -934,3 +934,11 @@ func (m *MockAdminClient) AddMockTask(task *types.TaskInput) {
func CreateAdminClient(adminServer string, workerID string, dialOption grpc.DialOption) (AdminClient, error) {
return NewGrpcAdminClient(adminServer, workerID, dialOption), nil
}
// getServerFromParams extracts server address from unified sources
func getServerFromParams(params *worker_pb.TaskParams) string {
if len(params.Sources) > 0 {
return params.Sources[0].Node
}
return ""
}

View File

@@ -0,0 +1,85 @@
package worker
import (
"fmt"
wtasks "github.com/seaweedfs/seaweedfs/weed/worker/tasks"
wtypes "github.com/seaweedfs/seaweedfs/weed/worker/types"
)
// taskLoggerAdapter adapts a tasks.TaskLogger to the types.Logger interface used by tasks
// so that structured WithFields logs from task implementations are captured into file logs.
type taskLoggerAdapter struct {
base wtasks.TaskLogger
fields map[string]interface{}
}
func newTaskLoggerAdapter(base wtasks.TaskLogger) *taskLoggerAdapter {
return &taskLoggerAdapter{base: base}
}
// WithFields returns a new adapter instance that includes the provided fields.
func (a *taskLoggerAdapter) WithFields(fields map[string]interface{}) wtypes.Logger {
// copy fields to avoid mutation by caller
copied := make(map[string]interface{}, len(fields))
for k, v := range fields {
copied[k] = v
}
return &taskLoggerAdapter{base: a.base, fields: copied}
}
// Info logs an info message, including any structured fields if present.
func (a *taskLoggerAdapter) Info(msg string, args ...interface{}) {
if a.base == nil {
return
}
if len(a.fields) > 0 {
a.base.LogWithFields("INFO", fmt.Sprintf(msg, args...), toStringMap(a.fields))
return
}
a.base.Info(msg, args...)
}
func (a *taskLoggerAdapter) Warning(msg string, args ...interface{}) {
if a.base == nil {
return
}
if len(a.fields) > 0 {
a.base.LogWithFields("WARNING", fmt.Sprintf(msg, args...), toStringMap(a.fields))
return
}
a.base.Warning(msg, args...)
}
func (a *taskLoggerAdapter) Error(msg string, args ...interface{}) {
if a.base == nil {
return
}
if len(a.fields) > 0 {
a.base.LogWithFields("ERROR", fmt.Sprintf(msg, args...), toStringMap(a.fields))
return
}
a.base.Error(msg, args...)
}
func (a *taskLoggerAdapter) Debug(msg string, args ...interface{}) {
if a.base == nil {
return
}
if len(a.fields) > 0 {
a.base.LogWithFields("DEBUG", fmt.Sprintf(msg, args...), toStringMap(a.fields))
return
}
a.base.Debug(msg, args...)
}
// toStringMap converts map[string]interface{} to map[string]interface{} where values are printable.
// The underlying tasks.TaskLogger handles arbitrary JSON values, but our gRPC conversion later
// expects strings; we rely on existing conversion there. Here we keep interface{} to preserve detail.
func toStringMap(in map[string]interface{}) map[string]interface{} {
out := make(map[string]interface{}, len(in))
for k, v := range in {
out[k] = v
}
return out
}

View File

@@ -48,21 +48,32 @@ func (t *BalanceTask) Execute(ctx context.Context, params *worker_pb.TaskParams)
return fmt.Errorf("balance parameters are required")
}
// Get planned destination
destNode := balanceParams.DestNode
// Get source and destination from unified arrays
if len(params.Sources) == 0 {
return fmt.Errorf("source is required for balance task")
}
if len(params.Targets) == 0 {
return fmt.Errorf("target is required for balance task")
}
sourceNode := params.Sources[0].Node
destNode := params.Targets[0].Node
if sourceNode == "" {
return fmt.Errorf("source node is required for balance task")
}
if destNode == "" {
return fmt.Errorf("destination node is required for balance task")
}
t.GetLogger().WithFields(map[string]interface{}{
"volume_id": t.volumeID,
"source": t.server,
"source": sourceNode,
"destination": destNode,
"collection": t.collection,
}).Info("Starting balance task - moving volume")
sourceServer := pb.ServerAddress(t.server)
sourceServer := pb.ServerAddress(sourceNode)
targetServer := pb.ServerAddress(destNode)
volumeId := needle.VolumeId(t.volumeID)
@@ -130,8 +141,16 @@ func (t *BalanceTask) Validate(params *worker_pb.TaskParams) error {
return fmt.Errorf("volume ID mismatch: expected %d, got %d", t.volumeID, params.VolumeId)
}
if params.Server != t.server {
return fmt.Errorf("source server mismatch: expected %s, got %s", t.server, params.Server)
// Validate that at least one source matches our server
found := false
for _, source := range params.Sources {
if source.Node == t.server {
found = true
break
}
}
if !found {
return fmt.Errorf("no source matches expected server %s", t.server)
}
return nil

View File

@@ -105,36 +105,54 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
return nil, nil // Skip this task if destination planning fails
}
// Create typed parameters with destination information
task.TypedParams = &worker_pb.TaskParams{
TaskId: taskID, // Link to ActiveTopology pending task
VolumeId: selectedVolume.VolumeID,
Server: selectedVolume.Server,
Collection: selectedVolume.Collection,
VolumeSize: selectedVolume.Size, // Store original volume size for tracking changes
TaskParams: &worker_pb.TaskParams_BalanceParams{
BalanceParams: &worker_pb.BalanceTaskParams{
DestNode: destinationPlan.TargetNode,
EstimatedSize: destinationPlan.ExpectedSize,
PlacementScore: destinationPlan.PlacementScore,
PlacementConflicts: destinationPlan.Conflicts,
ForceMove: false,
TimeoutSeconds: 600, // 10 minutes default
},
},
}
glog.V(1).Infof("Planned balance destination for volume %d: %s -> %s (score: %.2f)",
selectedVolume.VolumeID, selectedVolume.Server, destinationPlan.TargetNode, destinationPlan.PlacementScore)
// Add pending balance task to ActiveTopology for capacity management
// Find the actual disk containing the volume on the source server
sourceDisk, found := base.FindVolumeDisk(clusterInfo.ActiveTopology, selectedVolume.VolumeID, selectedVolume.Collection, selectedVolume.Server)
if !found {
return nil, fmt.Errorf("BALANCE: Could not find volume %d (collection: %s) on source server %s - unable to create balance task",
selectedVolume.VolumeID, selectedVolume.Collection, selectedVolume.Server)
}
// Create typed parameters with unified source and target information
task.TypedParams = &worker_pb.TaskParams{
TaskId: taskID, // Link to ActiveTopology pending task
VolumeId: selectedVolume.VolumeID,
Collection: selectedVolume.Collection,
VolumeSize: selectedVolume.Size, // Store original volume size for tracking changes
// Unified sources and targets - the only way to specify locations
Sources: []*worker_pb.TaskSource{
{
Node: selectedVolume.Server,
DiskId: sourceDisk,
VolumeId: selectedVolume.VolumeID,
EstimatedSize: selectedVolume.Size,
DataCenter: selectedVolume.DataCenter,
Rack: selectedVolume.Rack,
},
},
Targets: []*worker_pb.TaskTarget{
{
Node: destinationPlan.TargetNode,
DiskId: destinationPlan.TargetDisk,
VolumeId: selectedVolume.VolumeID,
EstimatedSize: destinationPlan.ExpectedSize,
DataCenter: destinationPlan.TargetDC,
Rack: destinationPlan.TargetRack,
},
},
TaskParams: &worker_pb.TaskParams_BalanceParams{
BalanceParams: &worker_pb.BalanceTaskParams{
ForceMove: false,
TimeoutSeconds: 600, // 10 minutes default
},
},
}
glog.V(1).Infof("Planned balance destination for volume %d: %s -> %s",
selectedVolume.VolumeID, selectedVolume.Server, destinationPlan.TargetNode)
// Add pending balance task to ActiveTopology for capacity management
targetDisk := destinationPlan.TargetDisk
err = clusterInfo.ActiveTopology.AddPendingTask(topology.TaskSpec{
@@ -220,7 +238,6 @@ func planBalanceDestination(activeTopology *topology.ActiveTopology, selectedVol
TargetDC: bestDisk.DataCenter,
ExpectedSize: selectedVolume.Size,
PlacementScore: bestScore,
Conflicts: checkPlacementConflicts(bestDisk, sourceRack, sourceDC),
}, nil
}
@@ -253,16 +270,3 @@ func calculateBalanceScore(disk *topology.DiskInfo, sourceRack, sourceDC string,
return score
}
// checkPlacementConflicts checks for placement rule conflicts
func checkPlacementConflicts(disk *topology.DiskInfo, sourceRack, sourceDC string) []string {
var conflicts []string
// For now, implement basic conflict detection
// This could be extended with more sophisticated placement rules
if disk.Rack == sourceRack && disk.DataCenter == sourceDC {
conflicts = append(conflicts, "same_rack_as_source")
}
return conflicts
}

View File

@@ -15,15 +15,13 @@ type TypedTask struct {
*base.BaseTypedTask
// Task state from protobuf
sourceServer string
destNode string
volumeID uint32
collection string
estimatedSize uint64
placementScore float64
forceMove bool
timeoutSeconds int32
placementConflicts []string
sourceServer string
destNode string
volumeID uint32
collection string
estimatedSize uint64
forceMove bool
timeoutSeconds int32
}
// NewTypedTask creates a new typed balance task
@@ -47,14 +45,20 @@ func (t *TypedTask) ValidateTyped(params *worker_pb.TaskParams) error {
return fmt.Errorf("balance_params is required for balance task")
}
// Validate destination node
if balanceParams.DestNode == "" {
return fmt.Errorf("dest_node is required for balance task")
// Validate sources and targets
if len(params.Sources) == 0 {
return fmt.Errorf("at least one source is required for balance task")
}
if len(params.Targets) == 0 {
return fmt.Errorf("at least one target is required for balance task")
}
// Validate estimated size
if balanceParams.EstimatedSize == 0 {
return fmt.Errorf("estimated_size must be greater than 0")
// Validate that source and target have volume IDs
if params.Sources[0].VolumeId == 0 {
return fmt.Errorf("source volume_id is required for balance task")
}
if params.Targets[0].VolumeId == 0 {
return fmt.Errorf("target volume_id is required for balance task")
}
// Validate timeout
@@ -73,10 +77,13 @@ func (t *TypedTask) EstimateTimeTyped(params *worker_pb.TaskParams) time.Duratio
if balanceParams.TimeoutSeconds > 0 {
return time.Duration(balanceParams.TimeoutSeconds) * time.Second
}
}
// Estimate based on volume size (1 minute per GB)
if balanceParams.EstimatedSize > 0 {
gbSize := balanceParams.EstimatedSize / (1024 * 1024 * 1024)
// Estimate based on volume size from sources (1 minute per GB)
if len(params.Sources) > 0 {
source := params.Sources[0]
if source.EstimatedSize > 0 {
gbSize := source.EstimatedSize / (1024 * 1024 * 1024)
return time.Duration(gbSize) * time.Minute
}
}
@@ -89,35 +96,30 @@ func (t *TypedTask) EstimateTimeTyped(params *worker_pb.TaskParams) time.Duratio
func (t *TypedTask) ExecuteTyped(params *worker_pb.TaskParams) error {
// Extract basic parameters
t.volumeID = params.VolumeId
t.sourceServer = params.Server
t.collection = params.Collection
// Ensure sources and targets are present (should be guaranteed by validation)
if len(params.Sources) == 0 {
return fmt.Errorf("at least one source is required for balance task (ExecuteTyped)")
}
if len(params.Targets) == 0 {
return fmt.Errorf("at least one target is required for balance task (ExecuteTyped)")
}
// Extract source and target information
t.sourceServer = params.Sources[0].Node
t.estimatedSize = params.Sources[0].EstimatedSize
t.destNode = params.Targets[0].Node
// Extract balance-specific parameters
balanceParams := params.GetBalanceParams()
if balanceParams != nil {
t.destNode = balanceParams.DestNode
t.estimatedSize = balanceParams.EstimatedSize
t.placementScore = balanceParams.PlacementScore
t.forceMove = balanceParams.ForceMove
t.timeoutSeconds = balanceParams.TimeoutSeconds
t.placementConflicts = balanceParams.PlacementConflicts
}
glog.Infof("Starting typed balance task for volume %d: %s -> %s (collection: %s, size: %d bytes)",
t.volumeID, t.sourceServer, t.destNode, t.collection, t.estimatedSize)
// Log placement information
if t.placementScore > 0 {
glog.V(1).Infof("Placement score: %.2f", t.placementScore)
}
if len(t.placementConflicts) > 0 {
glog.V(1).Infof("Placement conflicts: %v", t.placementConflicts)
if !t.forceMove {
return fmt.Errorf("placement conflicts detected and force_move is false: %v", t.placementConflicts)
}
glog.Warningf("Proceeding with balance despite conflicts (force_move=true): %v", t.placementConflicts)
}
// Simulate balance operation with progress updates
steps := []struct {
name string

View File

@@ -42,9 +42,12 @@ func RegisterBalanceTask() {
if params == nil {
return nil, fmt.Errorf("task parameters are required")
}
if len(params.Sources) == 0 {
return nil, fmt.Errorf("at least one source is required for balance task")
}
return NewBalanceTask(
fmt.Sprintf("balance-%d", params.VolumeId),
params.Server,
params.Sources[0].Node, // Use first source node
params.VolumeId,
params.Collection,
), nil

View File

@@ -16,7 +16,8 @@ type BaseTypedTask struct {
taskType types.TaskType
taskID string
progress float64
progressCallback func(float64)
progressCallback func(float64, string)
currentStage string
cancelled bool
mutex sync.RWMutex
@@ -75,21 +76,49 @@ func (bt *BaseTypedTask) GetProgress() float64 {
func (bt *BaseTypedTask) SetProgress(progress float64) {
bt.mutex.Lock()
callback := bt.progressCallback
stage := bt.currentStage
bt.progress = progress
bt.mutex.Unlock()
if callback != nil {
callback(progress)
callback(progress, stage)
}
}
// SetProgressCallback sets the progress callback function
func (bt *BaseTypedTask) SetProgressCallback(callback func(float64)) {
func (bt *BaseTypedTask) SetProgressCallback(callback func(float64, string)) {
bt.mutex.Lock()
defer bt.mutex.Unlock()
bt.progressCallback = callback
}
// SetProgressWithStage sets the current progress with a stage description
func (bt *BaseTypedTask) SetProgressWithStage(progress float64, stage string) {
bt.mutex.Lock()
callback := bt.progressCallback
bt.progress = progress
bt.currentStage = stage
bt.mutex.Unlock()
if callback != nil {
callback(progress, stage)
}
}
// SetCurrentStage sets the current stage description
func (bt *BaseTypedTask) SetCurrentStage(stage string) {
bt.mutex.Lock()
defer bt.mutex.Unlock()
bt.currentStage = stage
}
// GetCurrentStage returns the current stage description
func (bt *BaseTypedTask) GetCurrentStage() string {
bt.mutex.RLock()
defer bt.mutex.RUnlock()
return bt.currentStage
}
// SetLoggerConfig sets the logger configuration for this task
func (bt *BaseTypedTask) SetLoggerConfig(config types.TaskLoggerConfig) {
bt.mutex.Lock()
@@ -200,8 +229,8 @@ func (bt *BaseTypedTask) ValidateTyped(params *worker_pb.TaskParams) error {
if params.VolumeId == 0 {
return errors.New("volume_id is required")
}
if params.Server == "" {
return errors.New("server is required")
if len(params.Sources) == 0 {
return errors.New("at least one source is required")
}
return nil
}

View File

@@ -61,6 +61,8 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
// Check quiet duration and fullness criteria
if metric.Age >= quietThreshold && metric.FullnessRatio >= ecConfig.FullnessRatio {
glog.Infof("EC Detection: Volume %d meets all criteria, attempting to create task", metric.VolumeID)
// Generate task ID for ActiveTopology integration
taskID := fmt.Sprintf("ec_vol_%d_%d", metric.VolumeID, now.Unix())
@@ -79,11 +81,13 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
// Plan EC destinations if ActiveTopology is available
if clusterInfo.ActiveTopology != nil {
glog.Infof("EC Detection: ActiveTopology available, planning destinations for volume %d", metric.VolumeID)
multiPlan, err := planECDestinations(clusterInfo.ActiveTopology, metric, ecConfig)
if err != nil {
glog.Warningf("Failed to plan EC destinations for volume %d: %v", metric.VolumeID, err)
continue // Skip this volume if destination planning fails
}
glog.Infof("EC Detection: Successfully planned %d destinations for volume %d", len(multiPlan.Plans), metric.VolumeID)
// Calculate expected shard size for EC operation
// Each data shard will be approximately volumeSize / dataShards
@@ -100,23 +104,27 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
}
// Find all volume replica locations (server + disk) from topology
glog.Infof("EC Detection: Looking for replica locations for volume %d", metric.VolumeID)
replicaLocations := findVolumeReplicaLocations(clusterInfo.ActiveTopology, metric.VolumeID, metric.Collection)
if len(replicaLocations) == 0 {
glog.Warningf("No replica locations found for volume %d, skipping EC", metric.VolumeID)
continue
}
glog.Infof("EC Detection: Found %d replica locations for volume %d", len(replicaLocations), metric.VolumeID)
// Find existing EC shards from previous failed attempts
existingECShards := findExistingECShards(clusterInfo.ActiveTopology, metric.VolumeID, metric.Collection)
// Combine volume replicas and existing EC shards for cleanup
var allSourceLocations []topology.TaskSourceLocation
var sources []topology.TaskSourceSpec
// Add volume replicas (will free volume slots)
for _, replica := range replicaLocations {
allSourceLocations = append(allSourceLocations, topology.TaskSourceLocation{
sources = append(sources, topology.TaskSourceSpec{
ServerID: replica.ServerID,
DiskID: replica.DiskID,
DataCenter: replica.DataCenter,
Rack: replica.Rack,
CleanupType: topology.CleanupVolumeReplica,
})
}
@@ -131,9 +139,11 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
for _, shard := range existingECShards {
key := fmt.Sprintf("%s:%d", shard.ServerID, shard.DiskID)
if !duplicateCheck[key] { // Avoid duplicates if EC shards are on same disk as volume replicas
allSourceLocations = append(allSourceLocations, topology.TaskSourceLocation{
sources = append(sources, topology.TaskSourceSpec{
ServerID: shard.ServerID,
DiskID: shard.DiskID,
DataCenter: shard.DataCenter,
Rack: shard.Rack,
CleanupType: topology.CleanupECShards,
})
duplicateCheck[key] = true
@@ -141,17 +151,7 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
}
glog.V(2).Infof("Found %d volume replicas and %d existing EC shards for volume %d (total %d cleanup sources)",
len(replicaLocations), len(existingECShards), metric.VolumeID, len(allSourceLocations))
// Convert TaskSourceLocation to TaskSourceSpec
sources := make([]topology.TaskSourceSpec, len(allSourceLocations))
for i, srcLoc := range allSourceLocations {
sources[i] = topology.TaskSourceSpec{
ServerID: srcLoc.ServerID,
DiskID: srcLoc.DiskID,
CleanupType: srcLoc.CleanupType,
}
}
len(replicaLocations), len(existingECShards), metric.VolumeID, len(sources))
// Convert shard destinations to TaskDestinationSpec
destinations := make([]topology.TaskDestinationSpec, len(shardDestinations))
@@ -180,27 +180,21 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
}
glog.V(2).Infof("Added pending EC shard task %s to ActiveTopology for volume %d with %d cleanup sources and %d shard destinations",
taskID, metric.VolumeID, len(allSourceLocations), len(multiPlan.Plans))
taskID, metric.VolumeID, len(sources), len(multiPlan.Plans))
// Find all volume replicas from topology (for legacy worker compatibility)
var replicas []string
serverSet := make(map[string]struct{})
for _, loc := range replicaLocations {
if _, found := serverSet[loc.ServerID]; !found {
replicas = append(replicas, loc.ServerID)
serverSet[loc.ServerID] = struct{}{}
}
}
glog.V(1).Infof("Found %d replicas for volume %d: %v", len(replicas), metric.VolumeID, replicas)
// Create typed parameters with EC destination information and replicas
// Create unified sources and targets for EC task
result.TypedParams = &worker_pb.TaskParams{
TaskId: taskID, // Link to ActiveTopology pending task
VolumeId: metric.VolumeID,
Server: metric.Server,
Collection: metric.Collection,
VolumeSize: metric.Size, // Store original volume size for tracking changes
Replicas: replicas, // Include all volume replicas for deletion
// Unified sources - all sources that will be processed/cleaned up
Sources: convertTaskSourcesToProtobuf(sources, metric.VolumeID),
// Unified targets - all EC shard destinations
Targets: createECTargets(multiPlan),
TaskParams: &worker_pb.TaskParams_ErasureCodingParams{
ErasureCodingParams: createECTaskParams(multiPlan),
},
@@ -213,6 +207,7 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
continue // Skip this volume if no topology available
}
glog.Infof("EC Detection: Successfully created EC task for volume %d, adding to results", metric.VolumeID)
results = append(results, result)
} else {
// Count debug reasons
@@ -283,7 +278,8 @@ func planECDestinations(activeTopology *topology.ActiveTopology, metric *types.V
// Get available disks for EC placement with effective capacity consideration (includes pending tasks)
// For EC, we typically need 1 volume slot per shard, so use minimum capacity of 1
// For EC, we need at least 1 available volume slot on a disk to consider it for placement.
availableDisks := activeTopology.GetDisksWithEffectiveCapacity(topology.TaskTypeErasureCoding, metric.Server, 1)
// Note: We don't exclude the source server since the original volume will be deleted after EC conversion
availableDisks := activeTopology.GetDisksWithEffectiveCapacity(topology.TaskTypeErasureCoding, "", 1)
if len(availableDisks) < erasure_coding.MinTotalDisks {
return nil, fmt.Errorf("insufficient disks for EC placement: need %d, have %d (considering pending/active tasks)", erasure_coding.MinTotalDisks, len(availableDisks))
}
@@ -306,7 +302,6 @@ func planECDestinations(activeTopology *topology.ActiveTopology, metric *types.V
TargetDC: disk.DataCenter,
ExpectedSize: expectedShardSize, // Set calculated EC shard size
PlacementScore: calculateECScore(disk, sourceRack, sourceDC),
Conflicts: checkECPlacementConflicts(disk, sourceRack, sourceDC),
}
plans = append(plans, plan)
@@ -340,32 +335,96 @@ func planECDestinations(activeTopology *topology.ActiveTopology, metric *types.V
}, nil
}
// createECTaskParams creates EC task parameters from the multi-destination plan
func createECTaskParams(multiPlan *topology.MultiDestinationPlan) *worker_pb.ErasureCodingTaskParams {
var destinations []*worker_pb.ECDestination
// createECTargets creates unified TaskTarget structures from the multi-destination plan
// with proper shard ID assignment during planning phase
func createECTargets(multiPlan *topology.MultiDestinationPlan) []*worker_pb.TaskTarget {
var targets []*worker_pb.TaskTarget
numTargets := len(multiPlan.Plans)
for _, plan := range multiPlan.Plans {
destination := &worker_pb.ECDestination{
Node: plan.TargetNode,
DiskId: plan.TargetDisk,
Rack: plan.TargetRack,
DataCenter: plan.TargetDC,
PlacementScore: plan.PlacementScore,
// Create shard assignment arrays for each target (round-robin distribution)
targetShards := make([][]uint32, numTargets)
for i := range targetShards {
targetShards[i] = make([]uint32, 0)
}
// Distribute shards in round-robin fashion to spread both data and parity shards
// This ensures each target gets a mix of data shards (0-9) and parity shards (10-13)
for shardId := uint32(0); shardId < uint32(erasure_coding.TotalShardsCount); shardId++ {
targetIndex := int(shardId) % numTargets
targetShards[targetIndex] = append(targetShards[targetIndex], shardId)
}
// Create targets with assigned shard IDs
for i, plan := range multiPlan.Plans {
target := &worker_pb.TaskTarget{
Node: plan.TargetNode,
DiskId: plan.TargetDisk,
Rack: plan.TargetRack,
DataCenter: plan.TargetDC,
ShardIds: targetShards[i], // Round-robin assigned shards
EstimatedSize: plan.ExpectedSize,
}
destinations = append(destinations, destination)
targets = append(targets, target)
// Log shard assignment with data/parity classification
dataShards := make([]uint32, 0)
parityShards := make([]uint32, 0)
for _, shardId := range targetShards[i] {
if shardId < uint32(erasure_coding.DataShardsCount) {
dataShards = append(dataShards, shardId)
} else {
parityShards = append(parityShards, shardId)
}
}
glog.V(2).Infof("EC planning: target %s assigned shards %v (data: %v, parity: %v)",
plan.TargetNode, targetShards[i], dataShards, parityShards)
}
// Collect placement conflicts from all destinations
var placementConflicts []string
for _, plan := range multiPlan.Plans {
placementConflicts = append(placementConflicts, plan.Conflicts...)
glog.V(1).Infof("EC planning: distributed %d shards across %d targets using round-robin (data shards 0-%d, parity shards %d-%d)",
erasure_coding.TotalShardsCount, numTargets,
erasure_coding.DataShardsCount-1, erasure_coding.DataShardsCount, erasure_coding.TotalShardsCount-1)
return targets
}
// convertTaskSourcesToProtobuf converts topology.TaskSourceSpec to worker_pb.TaskSource
func convertTaskSourcesToProtobuf(sources []topology.TaskSourceSpec, volumeID uint32) []*worker_pb.TaskSource {
var protobufSources []*worker_pb.TaskSource
for _, source := range sources {
pbSource := &worker_pb.TaskSource{
Node: source.ServerID,
DiskId: source.DiskID,
DataCenter: source.DataCenter,
Rack: source.Rack,
}
// Convert storage impact to estimated size
if source.EstimatedSize != nil {
pbSource.EstimatedSize = uint64(*source.EstimatedSize)
}
// Set appropriate volume ID or shard IDs based on cleanup type
switch source.CleanupType {
case topology.CleanupVolumeReplica:
// This is a volume replica, use the actual volume ID
pbSource.VolumeId = volumeID
case topology.CleanupECShards:
// This is EC shards, also use the volume ID for consistency
pbSource.VolumeId = volumeID
// Note: ShardIds would need to be passed separately if we need specific shard info
}
protobufSources = append(protobufSources, pbSource)
}
return protobufSources
}
// createECTaskParams creates clean EC task parameters (destinations now in unified targets)
func createECTaskParams(multiPlan *topology.MultiDestinationPlan) *worker_pb.ErasureCodingTaskParams {
return &worker_pb.ErasureCodingTaskParams{
Destinations: destinations,
DataShards: erasure_coding.DataShardsCount, // Standard data shards
ParityShards: erasure_coding.ParityShardsCount, // Standard parity shards
PlacementConflicts: placementConflicts,
DataShards: erasure_coding.DataShardsCount, // Standard data shards
ParityShards: erasure_coding.ParityShardsCount, // Standard parity shards
}
}
@@ -456,25 +515,19 @@ func calculateECScore(disk *topology.DiskInfo, sourceRack, sourceDC string) floa
score := 0.0
// Prefer disks with available capacity
// Prefer disks with available capacity (primary factor)
if disk.DiskInfo.MaxVolumeCount > 0 {
utilization := float64(disk.DiskInfo.VolumeCount) / float64(disk.DiskInfo.MaxVolumeCount)
score += (1.0 - utilization) * 50.0 // Up to 50 points for available capacity
score += (1.0 - utilization) * 60.0 // Up to 60 points for available capacity
}
// Prefer different racks for better distribution
if disk.Rack != sourceRack {
score += 30.0
}
// Prefer different data centers for better distribution
if disk.DataCenter != sourceDC {
score += 20.0
}
// Consider current load
// Consider current load (secondary factor)
score += (10.0 - float64(disk.LoadCount)) // Up to 10 points for low load
// Note: We don't penalize placing shards on the same rack/DC as source
// since the original volume will be deleted after EC conversion.
// This allows for better network efficiency and storage utilization.
return score
}
@@ -492,19 +545,6 @@ func isDiskSuitableForEC(disk *topology.DiskInfo) bool {
return true
}
// checkECPlacementConflicts checks for placement rule conflicts in EC operations
func checkECPlacementConflicts(disk *topology.DiskInfo, sourceRack, sourceDC string) []string {
var conflicts []string
// For EC, being on the same rack as source is often acceptable
// but we note it as potential conflict for monitoring
if disk.Rack == sourceRack && disk.DataCenter == sourceDC {
conflicts = append(conflicts, "same_rack_as_source")
}
return conflicts
}
// findVolumeReplicaLocations finds all replica locations (server + disk) for the specified volume
// Uses O(1) indexed lookup for optimal performance on large clusters.
func findVolumeReplicaLocations(activeTopology *topology.ActiveTopology, volumeID uint32, collection string) []topology.VolumeReplica {

View File

@@ -7,7 +7,6 @@ import (
"math"
"os"
"path/filepath"
"sort"
"strings"
"time"
@@ -36,9 +35,9 @@ type ErasureCodingTask struct {
// EC parameters
dataShards int32
parityShards int32
destinations []*worker_pb.ECDestination
shardAssignment map[string][]string // destination -> assigned shard types
replicas []string // volume replica servers for deletion
targets []*worker_pb.TaskTarget // Unified targets for EC shards
sources []*worker_pb.TaskSource // Unified sources for cleanup
shardAssignment map[string][]string // destination -> assigned shard types
}
// NewErasureCodingTask creates a new unified EC task instance
@@ -67,18 +66,43 @@ func (t *ErasureCodingTask) Execute(ctx context.Context, params *worker_pb.TaskP
t.dataShards = ecParams.DataShards
t.parityShards = ecParams.ParityShards
t.workDir = ecParams.WorkingDir
t.destinations = ecParams.Destinations
t.replicas = params.Replicas // Get replicas from task parameters
t.targets = params.Targets // Get unified targets
t.sources = params.Sources // Get unified sources
// Log detailed task information
t.GetLogger().WithFields(map[string]interface{}{
"volume_id": t.volumeID,
"server": t.server,
"collection": t.collection,
"data_shards": t.dataShards,
"parity_shards": t.parityShards,
"destinations": len(t.destinations),
"total_shards": t.dataShards + t.parityShards,
"targets": len(t.targets),
"sources": len(t.sources),
}).Info("Starting erasure coding task")
// Log detailed target server assignments
for i, target := range t.targets {
t.GetLogger().WithFields(map[string]interface{}{
"target_index": i,
"server": target.Node,
"shard_ids": target.ShardIds,
"shard_count": len(target.ShardIds),
}).Info("Target server shard assignment")
}
// Log source information
for i, source := range t.sources {
t.GetLogger().WithFields(map[string]interface{}{
"source_index": i,
"server": source.Node,
"volume_id": source.VolumeId,
"disk_id": source.DiskId,
"rack": source.Rack,
"data_center": source.DataCenter,
}).Info("Source server information")
}
// Use the working directory from task parameters, or fall back to a default
baseWorkDir := t.workDir
@@ -112,14 +136,14 @@ func (t *ErasureCodingTask) Execute(ctx context.Context, params *worker_pb.TaskP
}()
// Step 1: Mark volume readonly
t.ReportProgress(10.0)
t.ReportProgressWithStage(10.0, "Marking volume readonly")
t.GetLogger().Info("Marking volume readonly")
if err := t.markVolumeReadonly(); err != nil {
return fmt.Errorf("failed to mark volume readonly: %v", err)
}
// Step 2: Copy volume files to worker
t.ReportProgress(25.0)
t.ReportProgressWithStage(25.0, "Copying volume files to worker")
t.GetLogger().Info("Copying volume files to worker")
localFiles, err := t.copyVolumeFilesToWorker(taskWorkDir)
if err != nil {
@@ -127,7 +151,7 @@ func (t *ErasureCodingTask) Execute(ctx context.Context, params *worker_pb.TaskP
}
// Step 3: Generate EC shards locally
t.ReportProgress(40.0)
t.ReportProgressWithStage(40.0, "Generating EC shards locally")
t.GetLogger().Info("Generating EC shards locally")
shardFiles, err := t.generateEcShardsLocally(localFiles, taskWorkDir)
if err != nil {
@@ -135,27 +159,27 @@ func (t *ErasureCodingTask) Execute(ctx context.Context, params *worker_pb.TaskP
}
// Step 4: Distribute shards to destinations
t.ReportProgress(60.0)
t.ReportProgressWithStage(60.0, "Distributing EC shards to destinations")
t.GetLogger().Info("Distributing EC shards to destinations")
if err := t.distributeEcShards(shardFiles); err != nil {
return fmt.Errorf("failed to distribute EC shards: %v", err)
}
// Step 5: Mount EC shards
t.ReportProgress(80.0)
t.ReportProgressWithStage(80.0, "Mounting EC shards")
t.GetLogger().Info("Mounting EC shards")
if err := t.mountEcShards(); err != nil {
return fmt.Errorf("failed to mount EC shards: %v", err)
}
// Step 6: Delete original volume
t.ReportProgress(90.0)
t.ReportProgressWithStage(90.0, "Deleting original volume")
t.GetLogger().Info("Deleting original volume")
if err := t.deleteOriginalVolume(); err != nil {
return fmt.Errorf("failed to delete original volume: %v", err)
}
t.ReportProgress(100.0)
t.ReportProgressWithStage(100.0, "EC processing complete")
glog.Infof("EC task completed successfully: volume %d from %s with %d shards distributed",
t.volumeID, t.server, len(shardFiles))
@@ -177,8 +201,16 @@ func (t *ErasureCodingTask) Validate(params *worker_pb.TaskParams) error {
return fmt.Errorf("volume ID mismatch: expected %d, got %d", t.volumeID, params.VolumeId)
}
if params.Server != t.server {
return fmt.Errorf("source server mismatch: expected %s, got %s", t.server, params.Server)
// Validate that at least one source matches our server
found := false
for _, source := range params.Sources {
if source.Node == t.server {
found = true
break
}
}
if !found {
return fmt.Errorf("no source matches expected server %s", t.server)
}
if ecParams.DataShards < 1 {
@@ -189,8 +221,8 @@ func (t *ErasureCodingTask) Validate(params *worker_pb.TaskParams) error {
return fmt.Errorf("invalid parity shards: %d (must be >= 1)", ecParams.ParityShards)
}
if len(ecParams.Destinations) < int(ecParams.DataShards+ecParams.ParityShards) {
return fmt.Errorf("insufficient destinations: got %d, need %d", len(ecParams.Destinations), ecParams.DataShards+ecParams.ParityShards)
if len(params.Targets) < int(ecParams.DataShards+ecParams.ParityShards) {
return fmt.Errorf("insufficient targets: got %d, need %d", len(params.Targets), ecParams.DataShards+ecParams.ParityShards)
}
return nil
@@ -224,6 +256,12 @@ func (t *ErasureCodingTask) markVolumeReadonly() error {
func (t *ErasureCodingTask) copyVolumeFilesToWorker(workDir string) (map[string]string, error) {
localFiles := make(map[string]string)
t.GetLogger().WithFields(map[string]interface{}{
"volume_id": t.volumeID,
"source": t.server,
"working_dir": workDir,
}).Info("Starting volume file copy from source server")
// Copy .dat file
datFile := filepath.Join(workDir, fmt.Sprintf("%d.dat", t.volumeID))
if err := t.copyFileFromSource(".dat", datFile); err != nil {
@@ -231,6 +269,16 @@ func (t *ErasureCodingTask) copyVolumeFilesToWorker(workDir string) (map[string]
}
localFiles["dat"] = datFile
// Log .dat file size
if info, err := os.Stat(datFile); err == nil {
t.GetLogger().WithFields(map[string]interface{}{
"file_type": ".dat",
"file_path": datFile,
"size_bytes": info.Size(),
"size_mb": float64(info.Size()) / (1024 * 1024),
}).Info("Volume data file copied successfully")
}
// Copy .idx file
idxFile := filepath.Join(workDir, fmt.Sprintf("%d.idx", t.volumeID))
if err := t.copyFileFromSource(".idx", idxFile); err != nil {
@@ -238,6 +286,16 @@ func (t *ErasureCodingTask) copyVolumeFilesToWorker(workDir string) (map[string]
}
localFiles["idx"] = idxFile
// Log .idx file size
if info, err := os.Stat(idxFile); err == nil {
t.GetLogger().WithFields(map[string]interface{}{
"file_type": ".idx",
"file_path": idxFile,
"size_bytes": info.Size(),
"size_mb": float64(info.Size()) / (1024 * 1024),
}).Info("Volume index file copied successfully")
}
return localFiles, nil
}
@@ -312,18 +370,38 @@ func (t *ErasureCodingTask) generateEcShardsLocally(localFiles map[string]string
return nil, fmt.Errorf("failed to generate .ecx file: %v", err)
}
// Collect generated shard file paths
// Collect generated shard file paths and log details
var generatedShards []string
var totalShardSize int64
for i := 0; i < erasure_coding.TotalShardsCount; i++ {
shardFile := fmt.Sprintf("%s.ec%02d", baseName, i)
if _, err := os.Stat(shardFile); err == nil {
shardFiles[fmt.Sprintf("ec%02d", i)] = shardFile
if info, err := os.Stat(shardFile); err == nil {
shardKey := fmt.Sprintf("ec%02d", i)
shardFiles[shardKey] = shardFile
generatedShards = append(generatedShards, shardKey)
totalShardSize += info.Size()
// Log individual shard details
t.GetLogger().WithFields(map[string]interface{}{
"shard_id": i,
"shard_type": shardKey,
"file_path": shardFile,
"size_bytes": info.Size(),
"size_kb": float64(info.Size()) / 1024,
}).Info("EC shard generated")
}
}
// Add metadata files
ecxFile := baseName + ".ecx"
if _, err := os.Stat(ecxFile); err == nil {
if info, err := os.Stat(ecxFile); err == nil {
shardFiles["ecx"] = ecxFile
t.GetLogger().WithFields(map[string]interface{}{
"file_type": "ecx",
"file_path": ecxFile,
"size_bytes": info.Size(),
}).Info("EC index file generated")
}
// Generate .vif file (volume info)
@@ -335,26 +413,67 @@ func (t *ErasureCodingTask) generateEcShardsLocally(localFiles map[string]string
glog.Warningf("Failed to create .vif file: %v", err)
} else {
shardFiles["vif"] = vifFile
if info, err := os.Stat(vifFile); err == nil {
t.GetLogger().WithFields(map[string]interface{}{
"file_type": "vif",
"file_path": vifFile,
"size_bytes": info.Size(),
}).Info("Volume info file generated")
}
}
glog.V(1).Infof("Generated %d EC files locally", len(shardFiles))
// Log summary of generation
t.GetLogger().WithFields(map[string]interface{}{
"total_files": len(shardFiles),
"ec_shards": len(generatedShards),
"generated_shards": generatedShards,
"total_shard_size_mb": float64(totalShardSize) / (1024 * 1024),
}).Info("EC shard generation completed")
return shardFiles, nil
}
// distributeEcShards distributes locally generated EC shards to destination servers
// using pre-assigned shard IDs from planning phase
func (t *ErasureCodingTask) distributeEcShards(shardFiles map[string]string) error {
if len(t.destinations) == 0 {
return fmt.Errorf("no destinations specified for EC shard distribution")
if len(t.targets) == 0 {
return fmt.Errorf("no targets specified for EC shard distribution")
}
if len(shardFiles) == 0 {
return fmt.Errorf("no shard files available for distribution")
}
// Create shard assignment: assign specific shards to specific destinations
shardAssignment := t.createShardAssignment(shardFiles)
// Build shard assignment from pre-assigned target shard IDs (from planning phase)
shardAssignment := make(map[string][]string)
for _, target := range t.targets {
if len(target.ShardIds) == 0 {
continue // Skip targets with no assigned shards
}
var assignedShards []string
// Convert shard IDs to shard file names (e.g., 0 → "ec00", 1 → "ec01")
for _, shardId := range target.ShardIds {
shardType := fmt.Sprintf("ec%02d", shardId)
assignedShards = append(assignedShards, shardType)
}
// Add metadata files (.ecx, .vif) to targets that have shards
if len(assignedShards) > 0 {
if _, hasEcx := shardFiles["ecx"]; hasEcx {
assignedShards = append(assignedShards, "ecx")
}
if _, hasVif := shardFiles["vif"]; hasVif {
assignedShards = append(assignedShards, "vif")
}
}
shardAssignment[target.Node] = assignedShards
}
if len(shardAssignment) == 0 {
return fmt.Errorf("failed to create shard assignment")
return fmt.Errorf("no shard assignments found from planning phase")
}
// Store assignment for use during mounting
@@ -365,102 +484,52 @@ func (t *ErasureCodingTask) distributeEcShards(shardFiles map[string]string) err
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"assigned_shards": len(assignedShards),
"shard_ids": assignedShards,
}).Info("Distributing assigned EC shards to destination")
"shard_types": assignedShards,
}).Info("Starting shard distribution to destination server")
// Send only the assigned shards to this destination
var transferredBytes int64
for _, shardType := range assignedShards {
filePath, exists := shardFiles[shardType]
if !exists {
return fmt.Errorf("shard file %s not found for destination %s", shardType, destNode)
}
// Log file size before transfer
if info, err := os.Stat(filePath); err == nil {
transferredBytes += info.Size()
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"shard_type": shardType,
"file_path": filePath,
"size_bytes": info.Size(),
"size_kb": float64(info.Size()) / 1024,
}).Info("Starting shard file transfer")
}
if err := t.sendShardFileToDestination(destNode, filePath, shardType); err != nil {
return fmt.Errorf("failed to send %s to %s: %v", shardType, destNode, err)
}
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"shard_type": shardType,
}).Info("Shard file transfer completed")
}
// Log summary for this destination
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"shards_transferred": len(assignedShards),
"total_bytes": transferredBytes,
"total_mb": float64(transferredBytes) / (1024 * 1024),
}).Info("All shards distributed to destination server")
}
glog.V(1).Infof("Successfully distributed EC shards to %d destinations", len(shardAssignment))
return nil
}
// createShardAssignment assigns specific EC shards to specific destination servers
// Each destination gets a subset of shards based on availability and placement rules
func (t *ErasureCodingTask) createShardAssignment(shardFiles map[string]string) map[string][]string {
assignment := make(map[string][]string)
// Collect all available EC shards (ec00-ec13)
var availableShards []string
for shardType := range shardFiles {
if strings.HasPrefix(shardType, "ec") && len(shardType) == 4 {
availableShards = append(availableShards, shardType)
}
}
// Sort shards for consistent assignment
sort.Strings(availableShards)
if len(availableShards) == 0 {
glog.Warningf("No EC shards found for assignment")
return assignment
}
// Calculate shards per destination
numDestinations := len(t.destinations)
if numDestinations == 0 {
return assignment
}
// Strategy: Distribute shards as evenly as possible across destinations
// With 14 shards and N destinations, some destinations get ⌈14/N⌉ shards, others get ⌊14/N⌋
shardsPerDest := len(availableShards) / numDestinations
extraShards := len(availableShards) % numDestinations
shardIndex := 0
for i, dest := range t.destinations {
var destShards []string
// Assign base number of shards
shardsToAssign := shardsPerDest
// Assign one extra shard to first 'extraShards' destinations
if i < extraShards {
shardsToAssign++
}
// Assign the shards
for j := 0; j < shardsToAssign && shardIndex < len(availableShards); j++ {
destShards = append(destShards, availableShards[shardIndex])
shardIndex++
}
assignment[dest.Node] = destShards
glog.V(2).Infof("Assigned shards %v to destination %s", destShards, dest.Node)
}
// Assign metadata files (.ecx, .vif) to each destination that has shards
// Note: .ecj files are created during mount, not during initial generation
for destNode, destShards := range assignment {
if len(destShards) > 0 {
// Add .ecx file if available
if _, hasEcx := shardFiles["ecx"]; hasEcx {
assignment[destNode] = append(assignment[destNode], "ecx")
}
// Add .vif file if available
if _, hasVif := shardFiles["vif"]; hasVif {
assignment[destNode] = append(assignment[destNode], "vif")
}
glog.V(2).Infof("Assigned metadata files (.ecx, .vif) to destination %s", destNode)
}
}
return assignment
}
// sendShardFileToDestination sends a single shard file to a destination server using ReceiveFile API
func (t *ErasureCodingTask) sendShardFileToDestination(destServer, filePath, shardType string) error {
return operation.WithVolumeServerClient(false, pb.ServerAddress(destServer), grpc.WithInsecure(),
@@ -565,6 +634,8 @@ func (t *ErasureCodingTask) mountEcShards() error {
for destNode, assignedShards := range t.shardAssignment {
// Convert shard names to shard IDs for mounting
var shardIds []uint32
var metadataFiles []string
for _, shardType := range assignedShards {
// Skip metadata files (.ecx, .vif) - only mount EC shards
if strings.HasPrefix(shardType, "ec") && len(shardType) == 4 {
@@ -573,16 +644,26 @@ func (t *ErasureCodingTask) mountEcShards() error {
if _, err := fmt.Sscanf(shardType[2:], "%d", &shardId); err == nil {
shardIds = append(shardIds, shardId)
}
} else {
metadataFiles = append(metadataFiles, shardType)
}
}
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"shard_ids": shardIds,
"shard_count": len(shardIds),
"metadata_files": metadataFiles,
}).Info("Starting EC shard mount operation")
if len(shardIds) == 0 {
glog.V(1).Infof("No EC shards to mount on %s (only metadata files)", destNode)
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"metadata_files": metadataFiles,
}).Info("No EC shards to mount (only metadata files)")
continue
}
glog.V(1).Infof("Mounting shards %v on %s", shardIds, destNode)
err := operation.WithVolumeServerClient(false, pb.ServerAddress(destNode), grpc.WithInsecure(),
func(client volume_server_pb.VolumeServerClient) error {
_, mountErr := client.VolumeEcShardsMount(context.Background(), &volume_server_pb.VolumeEcShardsMountRequest{
@@ -594,9 +675,18 @@ func (t *ErasureCodingTask) mountEcShards() error {
})
if err != nil {
glog.Warningf("Failed to mount shards %v on %s: %v", shardIds, destNode, err)
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"shard_ids": shardIds,
"error": err.Error(),
}).Error("Failed to mount EC shards")
} else {
glog.V(1).Infof("Successfully mounted EC shards %v on %s", shardIds, destNode)
t.GetLogger().WithFields(map[string]interface{}{
"destination": destNode,
"shard_ids": shardIds,
"volume_id": t.volumeID,
"collection": t.collection,
}).Info("Successfully mounted EC shards")
}
}
@@ -613,13 +703,24 @@ func (t *ErasureCodingTask) deleteOriginalVolume() error {
replicas = []string{t.server}
}
glog.V(1).Infof("Deleting volume %d from %d replica servers: %v", t.volumeID, len(replicas), replicas)
t.GetLogger().WithFields(map[string]interface{}{
"volume_id": t.volumeID,
"replica_count": len(replicas),
"replica_servers": replicas,
}).Info("Starting original volume deletion from replica servers")
// Delete volume from all replica locations
var deleteErrors []string
successCount := 0
for _, replicaServer := range replicas {
for i, replicaServer := range replicas {
t.GetLogger().WithFields(map[string]interface{}{
"replica_index": i + 1,
"total_replicas": len(replicas),
"server": replicaServer,
"volume_id": t.volumeID,
}).Info("Deleting volume from replica server")
err := operation.WithVolumeServerClient(false, pb.ServerAddress(replicaServer), grpc.WithInsecure(),
func(client volume_server_pb.VolumeServerClient) error {
_, err := client.VolumeDelete(context.Background(), &volume_server_pb.VolumeDeleteRequest{
@@ -631,27 +732,52 @@ func (t *ErasureCodingTask) deleteOriginalVolume() error {
if err != nil {
deleteErrors = append(deleteErrors, fmt.Sprintf("failed to delete volume %d from %s: %v", t.volumeID, replicaServer, err))
glog.Warningf("Failed to delete volume %d from replica server %s: %v", t.volumeID, replicaServer, err)
t.GetLogger().WithFields(map[string]interface{}{
"server": replicaServer,
"volume_id": t.volumeID,
"error": err.Error(),
}).Error("Failed to delete volume from replica server")
} else {
successCount++
glog.V(1).Infof("Successfully deleted volume %d from replica server %s", t.volumeID, replicaServer)
t.GetLogger().WithFields(map[string]interface{}{
"server": replicaServer,
"volume_id": t.volumeID,
}).Info("Successfully deleted volume from replica server")
}
}
// Report results
if len(deleteErrors) > 0 {
glog.Warningf("Some volume deletions failed (%d/%d successful): %v", successCount, len(replicas), deleteErrors)
t.GetLogger().WithFields(map[string]interface{}{
"volume_id": t.volumeID,
"successful": successCount,
"failed": len(deleteErrors),
"total_replicas": len(replicas),
"success_rate": float64(successCount) / float64(len(replicas)) * 100,
"errors": deleteErrors,
}).Warning("Some volume deletions failed")
// Don't return error - EC task should still be considered successful if shards are mounted
} else {
glog.V(1).Infof("Successfully deleted volume %d from all %d replica servers", t.volumeID, len(replicas))
t.GetLogger().WithFields(map[string]interface{}{
"volume_id": t.volumeID,
"replica_count": len(replicas),
"replica_servers": replicas,
}).Info("Successfully deleted volume from all replica servers")
}
return nil
}
// getReplicas extracts replica servers from task parameters
// getReplicas extracts replica servers from unified sources
func (t *ErasureCodingTask) getReplicas() []string {
// Access replicas from the parameters passed during Execute
// We'll need to store these during Execute - let me add a field to the task
return t.replicas
var replicas []string
for _, source := range t.sources {
// Only include volume replica sources (not EC shard sources)
// Assumption: VolumeId == 0 is considered invalid and should be excluded.
// If volume ID 0 is valid in some contexts, update this check accordingly.
if source.VolumeId > 0 {
replicas = append(replicas, source.Node)
}
}
return replicas
}

View File

@@ -42,9 +42,12 @@ func RegisterErasureCodingTask() {
if params == nil {
return nil, fmt.Errorf("task parameters are required")
}
if len(params.Sources) == 0 {
return nil, fmt.Errorf("at least one source is required for erasure coding task")
}
return NewErasureCodingTask(
fmt.Sprintf("erasure_coding-%d", params.VolumeId),
params.Server,
params.Sources[0].Node, // Use first source node
params.VolumeId,
params.Collection,
), nil

View File

@@ -7,6 +7,7 @@ import (
"time"
"github.com/seaweedfs/seaweedfs/weed/glog"
"github.com/seaweedfs/seaweedfs/weed/pb/worker_pb"
"github.com/seaweedfs/seaweedfs/weed/worker/types"
)
@@ -21,7 +22,8 @@ type BaseTask struct {
estimatedDuration time.Duration
logger TaskLogger
loggerConfig TaskLoggerConfig
progressCallback func(float64) // Callback function for progress updates
progressCallback func(float64, string) // Callback function for progress updates
currentStage string // Current stage description
}
// NewBaseTask creates a new base task
@@ -90,20 +92,64 @@ func (t *BaseTask) SetProgress(progress float64) {
}
oldProgress := t.progress
callback := t.progressCallback
stage := t.currentStage
t.progress = progress
t.mutex.Unlock()
// Log progress change
if t.logger != nil && progress != oldProgress {
t.logger.LogProgress(progress, fmt.Sprintf("Progress updated from %.1f%% to %.1f%%", oldProgress, progress))
message := stage
if message == "" {
message = fmt.Sprintf("Progress updated from %.1f%% to %.1f%%", oldProgress, progress)
}
t.logger.LogProgress(progress, message)
}
// Call progress callback if set
if callback != nil && progress != oldProgress {
callback(progress)
callback(progress, stage)
}
}
// SetProgressWithStage sets the current progress with a stage description
func (t *BaseTask) SetProgressWithStage(progress float64, stage string) {
t.mutex.Lock()
if progress < 0 {
progress = 0
}
if progress > 100 {
progress = 100
}
callback := t.progressCallback
t.progress = progress
t.currentStage = stage
t.mutex.Unlock()
// Log progress change
if t.logger != nil {
t.logger.LogProgress(progress, stage)
}
// Call progress callback if set
if callback != nil {
callback(progress, stage)
}
}
// SetCurrentStage sets the current stage description
func (t *BaseTask) SetCurrentStage(stage string) {
t.mutex.Lock()
defer t.mutex.Unlock()
t.currentStage = stage
}
// GetCurrentStage returns the current stage description
func (t *BaseTask) GetCurrentStage() string {
t.mutex.RLock()
defer t.mutex.RUnlock()
return t.currentStage
}
// Cancel cancels the task
func (t *BaseTask) Cancel() error {
t.mutex.Lock()
@@ -170,7 +216,7 @@ func (t *BaseTask) GetEstimatedDuration() time.Duration {
}
// SetProgressCallback sets the progress callback function
func (t *BaseTask) SetProgressCallback(callback func(float64)) {
func (t *BaseTask) SetProgressCallback(callback func(float64, string)) {
t.mutex.Lock()
defer t.mutex.Unlock()
t.progressCallback = callback
@@ -273,7 +319,7 @@ func (t *BaseTask) ExecuteTask(ctx context.Context, params types.TaskParams, exe
if t.logger != nil {
t.logger.LogWithFields("INFO", "Task execution started", map[string]interface{}{
"volume_id": params.VolumeID,
"server": params.Server,
"server": getServerFromSources(params.TypedParams.Sources),
"collection": params.Collection,
})
}
@@ -362,7 +408,7 @@ func ValidateParams(params types.TaskParams, requiredFields ...string) error {
return &ValidationError{Field: field, Message: "volume_id is required"}
}
case "server":
if params.Server == "" {
if len(params.TypedParams.Sources) == 0 {
return &ValidationError{Field: field, Message: "server is required"}
}
case "collection":
@@ -383,3 +429,11 @@ type ValidationError struct {
func (e *ValidationError) Error() string {
return e.Field + ": " + e.Message
}
// getServerFromSources extracts the server address from unified sources
func getServerFromSources(sources []*worker_pb.TaskSource) string {
if len(sources) > 0 {
return sources[0].Node
}
return ""
}

View File

@@ -5,6 +5,7 @@ import (
"os"
"path/filepath"
"strings"
"time"
"github.com/seaweedfs/seaweedfs/weed/glog"
"github.com/seaweedfs/seaweedfs/weed/pb/worker_pb"
@@ -20,6 +21,10 @@ func NewTaskLogHandler(baseLogDir string) *TaskLogHandler {
if baseLogDir == "" {
baseLogDir = "/tmp/seaweedfs/task_logs"
}
// Best-effort ensure the base directory exists so reads don't fail due to missing dir
if err := os.MkdirAll(baseLogDir, 0755); err != nil {
glog.Warningf("Failed to create base task log directory %s: %v", baseLogDir, err)
}
return &TaskLogHandler{
baseLogDir: baseLogDir,
}
@@ -38,6 +43,23 @@ func (h *TaskLogHandler) HandleLogRequest(request *worker_pb.TaskLogRequest) *wo
if err != nil {
response.ErrorMessage = fmt.Sprintf("Task log directory not found: %v", err)
glog.Warningf("Task log request failed for %s: %v", request.TaskId, err)
// Add diagnostic information to help debug the issue
response.LogEntries = []*worker_pb.TaskLogEntry{
{
Timestamp: time.Now().Unix(),
Level: "WARNING",
Message: fmt.Sprintf("Task logs not available: %v", err),
Fields: map[string]string{"source": "task_log_handler"},
},
{
Timestamp: time.Now().Unix(),
Level: "INFO",
Message: fmt.Sprintf("This usually means the task was never executed on this worker or logs were cleaned up. Base log directory: %s", h.baseLogDir),
Fields: map[string]string{"source": "task_log_handler"},
},
}
// response.Success remains false to indicate logs were not found
return response
}
@@ -71,17 +93,23 @@ func (h *TaskLogHandler) HandleLogRequest(request *worker_pb.TaskLogRequest) *wo
func (h *TaskLogHandler) findTaskLogDirectory(taskID string) (string, error) {
entries, err := os.ReadDir(h.baseLogDir)
if err != nil {
return "", fmt.Errorf("failed to read base log directory: %w", err)
return "", fmt.Errorf("failed to read base log directory %s: %w", h.baseLogDir, err)
}
// Look for directories that start with the task ID
var candidateDirs []string
for _, entry := range entries {
if entry.IsDir() && strings.HasPrefix(entry.Name(), taskID+"_") {
return filepath.Join(h.baseLogDir, entry.Name()), nil
if entry.IsDir() {
candidateDirs = append(candidateDirs, entry.Name())
if strings.HasPrefix(entry.Name(), taskID+"_") {
return filepath.Join(h.baseLogDir, entry.Name()), nil
}
}
}
return "", fmt.Errorf("task log directory not found for task ID: %s", taskID)
// Enhanced error message with diagnostic information
return "", fmt.Errorf("task log directory not found for task ID: %s (searched %d directories in %s, directories found: %v)",
taskID, len(candidateDirs), h.baseLogDir, candidateDirs)
}
// readTaskMetadata reads task metadata from the log directory

View File

@@ -127,7 +127,7 @@ func NewTaskLogger(taskID string, taskType types.TaskType, workerID string, para
Status: "started",
Progress: 0.0,
VolumeID: params.VolumeID,
Server: params.Server,
Server: getServerFromSources(params.TypedParams.Sources),
Collection: params.Collection,
CustomData: make(map[string]interface{}),
LogFilePath: logFilePath,
@@ -149,7 +149,7 @@ func NewTaskLogger(taskID string, taskType types.TaskType, workerID string, para
logger.Info("Task logger initialized for %s (type: %s, worker: %s)", taskID, taskType, workerID)
logger.LogWithFields("INFO", "Task parameters", map[string]interface{}{
"volume_id": params.VolumeID,
"server": params.Server,
"server": getServerFromSources(params.TypedParams.Sources),
"collection": params.Collection,
})

View File

@@ -47,7 +47,7 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
}
// Create typed parameters for vacuum task
result.TypedParams = createVacuumTaskParams(result, metric, vacuumConfig)
result.TypedParams = createVacuumTaskParams(result, metric, vacuumConfig, clusterInfo)
results = append(results, result)
} else {
// Debug why volume was not selected
@@ -85,7 +85,7 @@ func Detection(metrics []*types.VolumeHealthMetrics, clusterInfo *types.ClusterI
// createVacuumTaskParams creates typed parameters for vacuum tasks
// This function is moved from MaintenanceIntegration.createVacuumTaskParams to the detection logic
func createVacuumTaskParams(task *types.TaskDetectionResult, metric *types.VolumeHealthMetrics, vacuumConfig *Config) *worker_pb.TaskParams {
func createVacuumTaskParams(task *types.TaskDetectionResult, metric *types.VolumeHealthMetrics, vacuumConfig *Config, clusterInfo *types.ClusterInfo) *worker_pb.TaskParams {
// Use configured values or defaults
garbageThreshold := 0.3 // Default 30%
verifyChecksum := true // Default to verify
@@ -99,13 +99,27 @@ func createVacuumTaskParams(task *types.TaskDetectionResult, metric *types.Volum
// to the protobuf definition if they should be configurable
}
// Create typed protobuf parameters
// Use DC and rack information directly from VolumeHealthMetrics
sourceDC, sourceRack := metric.DataCenter, metric.Rack
// Create typed protobuf parameters with unified sources
return &worker_pb.TaskParams{
TaskId: task.TaskID, // Link to ActiveTopology pending task (if integrated)
VolumeId: task.VolumeID,
Server: task.Server,
Collection: task.Collection,
VolumeSize: metric.Size, // Store original volume size for tracking changes
// Unified sources array
Sources: []*worker_pb.TaskSource{
{
Node: task.Server,
VolumeId: task.VolumeID,
EstimatedSize: metric.Size,
DataCenter: sourceDC,
Rack: sourceRack,
},
},
TaskParams: &worker_pb.TaskParams_VacuumParams{
VacuumParams: &worker_pb.VacuumTaskParams{
GarbageThreshold: garbageThreshold,

View File

@@ -42,9 +42,12 @@ func RegisterVacuumTask() {
if params == nil {
return nil, fmt.Errorf("task parameters are required")
}
if len(params.Sources) == 0 {
return nil, fmt.Errorf("at least one source is required for vacuum task")
}
return NewVacuumTask(
fmt.Sprintf("vacuum-%d", params.VolumeId),
params.Server,
params.Sources[0].Node, // Use first source node
params.VolumeId,
params.Collection,
), nil

View File

@@ -114,8 +114,16 @@ func (t *VacuumTask) Validate(params *worker_pb.TaskParams) error {
return fmt.Errorf("volume ID mismatch: expected %d, got %d", t.volumeID, params.VolumeId)
}
if params.Server != t.server {
return fmt.Errorf("source server mismatch: expected %s, got %s", t.server, params.Server)
// Validate that at least one source matches our server
found := false
for _, source := range params.Sources {
if source.Node == t.server {
found = true
break
}
}
if !found {
return fmt.Errorf("no source matches expected server %s", t.server)
}
if vacuumParams.GarbageThreshold < 0 || vacuumParams.GarbageThreshold > 1.0 {

View File

@@ -12,9 +12,10 @@ import (
type BaseTask struct {
id string
taskType types.TaskType
progressCallback func(float64)
progressCallback func(float64, string) // Modified to include stage description
logger types.Logger
cancelled bool
currentStage string
}
// NewBaseTask creates a new base task
@@ -37,17 +38,35 @@ func (t *BaseTask) Type() types.TaskType {
}
// SetProgressCallback sets the progress callback
func (t *BaseTask) SetProgressCallback(callback func(float64)) {
func (t *BaseTask) SetProgressCallback(callback func(float64, string)) {
t.progressCallback = callback
}
// ReportProgress reports current progress through the callback
func (t *BaseTask) ReportProgress(progress float64) {
if t.progressCallback != nil {
t.progressCallback(progress)
t.progressCallback(progress, t.currentStage)
}
}
// ReportProgressWithStage reports current progress with a specific stage description
func (t *BaseTask) ReportProgressWithStage(progress float64, stage string) {
t.currentStage = stage
if t.progressCallback != nil {
t.progressCallback(progress, stage)
}
}
// SetCurrentStage sets the current stage description
func (t *BaseTask) SetCurrentStage(stage string) {
t.currentStage = stage
}
// GetCurrentStage returns the current stage description
func (t *BaseTask) GetCurrentStage() string {
return t.currentStage
}
// GetProgress returns current progress
func (t *BaseTask) GetProgress() float64 {
// Subclasses should override this

View File

@@ -21,6 +21,8 @@ type VolumeHealthMetrics struct {
Server string
DiskType string // Disk type (e.g., "hdd", "ssd") or disk path (e.g., "/data1")
DiskId uint32 // ID of the disk in Store.Locations array
DataCenter string // Data center of the server
Rack string // Rack of the server
Collection string
Size uint64
DeletedBytes uint64

View File

@@ -28,7 +28,7 @@ type Task interface {
// Progress
GetProgress() float64
SetProgressCallback(func(float64))
SetProgressCallback(func(float64, string))
}
// TaskWithLogging extends Task with logging capabilities
@@ -127,9 +127,10 @@ type LoggerFactory interface {
type UnifiedBaseTask struct {
id string
taskType TaskType
progressCallback func(float64)
progressCallback func(float64, string)
logger Logger
cancelled bool
currentStage string
}
// NewBaseTask creates a new base task
@@ -151,17 +152,35 @@ func (t *UnifiedBaseTask) Type() TaskType {
}
// SetProgressCallback sets the progress callback
func (t *UnifiedBaseTask) SetProgressCallback(callback func(float64)) {
func (t *UnifiedBaseTask) SetProgressCallback(callback func(float64, string)) {
t.progressCallback = callback
}
// ReportProgress reports current progress through the callback
func (t *UnifiedBaseTask) ReportProgress(progress float64) {
if t.progressCallback != nil {
t.progressCallback(progress)
t.progressCallback(progress, t.currentStage)
}
}
// ReportProgressWithStage reports current progress with a specific stage description
func (t *UnifiedBaseTask) ReportProgressWithStage(progress float64, stage string) {
t.currentStage = stage
if t.progressCallback != nil {
t.progressCallback(progress, stage)
}
}
// SetCurrentStage sets the current stage description
func (t *UnifiedBaseTask) SetCurrentStage(stage string) {
t.currentStage = stage
}
// GetCurrentStage returns the current stage description
func (t *UnifiedBaseTask) GetCurrentStage() string {
return t.currentStage
}
// Cancel marks the task as cancelled
func (t *UnifiedBaseTask) Cancel() error {
t.cancelled = true

View File

@@ -64,7 +64,6 @@ type TaskInput struct {
// TaskParams represents parameters for task execution
type TaskParams struct {
VolumeID uint32 `json:"volume_id,omitempty"`
Server string `json:"server,omitempty"`
Collection string `json:"collection,omitempty"`
WorkingDir string `json:"working_dir,omitempty"`
TypedParams *worker_pb.TaskParams `json:"typed_params,omitempty"`

View File

@@ -54,7 +54,7 @@ type TypedTaskInterface interface {
GetProgress() float64
// Set progress callback for progress updates
SetProgressCallback(callback func(float64))
SetProgressCallback(callback func(float64, string))
// Logger configuration and initialization (all typed tasks support this)
SetLoggerConfig(config TaskLoggerConfig)

View File

@@ -4,7 +4,6 @@ import (
"context"
"crypto/rand"
"fmt"
"net"
"os"
"path/filepath"
"strings"
@@ -78,43 +77,39 @@ func GenerateOrLoadWorkerID(workingDir string) (string, error) {
}
}
// Generate new unique worker ID with host information
// Generate simplified worker ID
hostname, _ := os.Hostname()
if hostname == "" {
hostname = "unknown"
}
// Get local IP address for better host identification
var hostIP string
if addrs, err := net.InterfaceAddrs(); err == nil {
for _, addr := range addrs {
if ipnet, ok := addr.(*net.IPNet); ok && !ipnet.IP.IsLoopback() {
if ipnet.IP.To4() != nil {
hostIP = ipnet.IP.String()
break
}
// Use short hostname - take first 6 chars or last part after dots
shortHostname := hostname
if len(hostname) > 6 {
if parts := strings.Split(hostname, "."); len(parts) > 1 {
// Use last part before domain (e.g., "worker1" from "worker1.example.com")
shortHostname = parts[0]
if len(shortHostname) > 6 {
shortHostname = shortHostname[:6]
}
} else {
// Use first 6 characters
shortHostname = hostname[:6]
}
}
if hostIP == "" {
hostIP = "noip"
}
// Create host identifier combining hostname and IP
hostID := fmt.Sprintf("%s@%s", hostname, hostIP)
// Generate random component for uniqueness
randomBytes := make([]byte, 4)
// Generate random component for uniqueness (2 bytes = 4 hex chars)
randomBytes := make([]byte, 2)
var workerID string
if _, err := rand.Read(randomBytes); err != nil {
// Fallback to timestamp if crypto/rand fails
workerID = fmt.Sprintf("worker-%s-%d", hostID, time.Now().Unix())
// Fallback to short timestamp if crypto/rand fails
timestamp := time.Now().Unix() % 10000 // last 4 digits
workerID = fmt.Sprintf("w-%s-%04d", shortHostname, timestamp)
glog.Infof("Generated fallback worker ID: %s", workerID)
} else {
// Use random bytes + timestamp for uniqueness
// Use random hex for uniqueness
randomHex := fmt.Sprintf("%x", randomBytes)
timestamp := time.Now().Unix()
workerID = fmt.Sprintf("worker-%s-%s-%d", hostID, randomHex, timestamp)
workerID = fmt.Sprintf("w-%s-%s", shortHostname, randomHex)
glog.Infof("Generated new worker ID: %s", workerID)
}
@@ -145,6 +140,10 @@ func NewWorker(config *types.WorkerConfig) (*Worker, error) {
// Initialize task log handler
logDir := filepath.Join(config.BaseWorkingDir, "task_logs")
// Ensure the base task log directory exists to avoid errors when admin requests logs
if err := os.MkdirAll(logDir, 0755); err != nil {
glog.Warningf("Failed to create task log base directory %s: %v", logDir, err)
}
taskLogHandler := tasks.NewTaskLogHandler(logDir)
worker := &Worker{
@@ -407,6 +406,26 @@ func (w *Worker) executeTask(task *types.TaskInput) {
// Use new task execution system with unified Task interface
glog.V(1).Infof("Executing task %s with typed protobuf parameters", task.ID)
// Initialize a file-based task logger so admin can retrieve logs
// Build minimal params for logger metadata
loggerParams := types.TaskParams{
VolumeID: task.VolumeID,
Collection: task.Collection,
TypedParams: task.TypedParams,
}
loggerConfig := w.getTaskLoggerConfig()
fileLogger, logErr := tasks.NewTaskLogger(task.ID, task.Type, w.id, loggerParams, loggerConfig)
if logErr != nil {
glog.Warningf("Failed to initialize file logger for task %s: %v", task.ID, logErr)
} else {
defer func() {
if err := fileLogger.Close(); err != nil {
glog.V(1).Infof("Failed to close task logger for %s: %v", task.ID, err)
}
}()
fileLogger.Info("Task %s started (type=%s, server=%s, collection=%s)", task.ID, task.Type, task.Server, task.Collection)
}
taskFactory := w.registry.Get(task.Type)
if taskFactory == nil {
w.completeTask(task.ID, false, fmt.Sprintf("task factory not available for %s: task type not found", task.Type))
@@ -431,13 +450,28 @@ func (w *Worker) executeTask(task *types.TaskInput) {
// Task execution uses the new unified Task interface
glog.V(2).Infof("Executing task %s in working directory: %s", task.ID, taskWorkingDir)
// If we have a file logger, adapt it so task WithFields logs are captured into file
if fileLogger != nil {
if withLogger, ok := taskInstance.(interface{ SetLogger(types.Logger) }); ok {
withLogger.SetLogger(newTaskLoggerAdapter(fileLogger))
}
}
// Set progress callback that reports to admin server
taskInstance.SetProgressCallback(func(progress float64) {
taskInstance.SetProgressCallback(func(progress float64, stage string) {
// Report progress updates to admin server
glog.V(2).Infof("Task %s progress: %.1f%%", task.ID, progress)
glog.V(2).Infof("Task %s progress: %.1f%% - %s", task.ID, progress, stage)
if err := w.adminClient.UpdateTaskProgress(task.ID, progress); err != nil {
glog.V(1).Infof("Failed to report task progress to admin: %v", err)
}
if fileLogger != nil {
// Use meaningful stage description or fallback to generic message
message := stage
if message == "" {
message = fmt.Sprintf("Progress: %.1f%%", progress)
}
fileLogger.LogProgress(progress, message)
}
})
// Execute task with context
@@ -449,10 +483,17 @@ func (w *Worker) executeTask(task *types.TaskInput) {
w.completeTask(task.ID, false, err.Error())
w.tasksFailed++
glog.Errorf("Worker %s failed to execute task %s: %v", w.id, task.ID, err)
if fileLogger != nil {
fileLogger.LogStatus("failed", err.Error())
fileLogger.Error("Task %s failed: %v", task.ID, err)
}
} else {
w.completeTask(task.ID, true, "")
w.tasksCompleted++
glog.Infof("Worker %s completed task %s successfully", w.id, task.ID)
if fileLogger != nil {
fileLogger.Info("Task %s completed successfully", task.ID)
}
}
}
@@ -696,7 +737,7 @@ func (w *Worker) processAdminMessage(message *worker_pb.AdminMessage) {
Type: types.TaskType(taskAssign.TaskType),
Status: types.TaskStatusAssigned,
VolumeID: taskAssign.Params.VolumeId,
Server: taskAssign.Params.Server,
Server: getServerFromParams(taskAssign.Params),
Collection: taskAssign.Params.Collection,
Priority: types.TaskPriority(taskAssign.Priority),
CreatedAt: time.Unix(taskAssign.CreatedTime, 0),