From 4416515996358b0950cdbd390060cbb476998f4f Mon Sep 17 00:00:00 2001 From: JamesMurkin Date: Thu, 7 Nov 2024 16:17:13 +0000 Subject: [PATCH 1/2] Add a reason for preemption (#3881) * [POC] Add a cause field to JobPreemptedEvent This will now be populated with a cause for the job to be preempted Allowing you to distinguish: - If this Preemption was caused by API request - If this Preemption was caused by Fair share preemption - and the preempting job id - If this Preemption was caused by Urgency based preemption - and the preempting job id(s) - We cannot always narrow this down to a single job, so it will be a comma separated list of job ids Signed-off-by: JamesMurkin * Proto changes Signed-off-by: JamesMurkin * Remove changes to loadtest.go Signed-off-by: JamesMurkin * Fix proto Signed-off-by: JamesMurkin * Fix proto Signed-off-by: JamesMurkin * Fix proto Signed-off-by: JamesMurkin * Generate proto Signed-off-by: JamesMurkin * Merge Signed-off-by: JamesMurkin * Fix common Signed-off-by: JamesMurkin * Add tests Signed-off-by: JamesMurkin * Add unit tests Signed-off-by: JamesMurkin * Improve tests Signed-off-by: JamesMurkin * Fix simulator Signed-off-by: JamesMurkin * Undo unrelated change Signed-off-by: JamesMurkin * Set PreemptingJobId Signed-off-by: JamesMurkin --------- Signed-off-by: JamesMurkin --- cmd/armadactl/cmd/preempt.go | 9 +- internal/armadactl/preempt.go | 3 +- internal/common/ingest/testfixtures/event.go | 2 + .../instructions/instructions.go | 2 +- .../instructions/instructions_test.go | 2 +- internal/scheduler/nodedb/nodedb.go | 7 + internal/scheduler/scheduler.go | 29 +- internal/scheduler/scheduling/context/job.go | 5 + internal/scheduler/scheduling/context/pod.go | 13 + .../scheduling/preempting_queue_scheduler.go | 1 + .../scheduling/preemption_description.go | 60 ++ .../scheduling/preemption_description_test.go | 141 ++++ internal/scheduler/simulator/simulator.go | 16 +- .../server/event/conversion/conversions.go | 4 +- .../event/conversion/conversions_test.go | 6 +- internal/server/submit/submit.go | 7 +- internal/server/submit/submit_test.go | 4 +- .../submit/testfixtures/test_fixtures.go | 5 +- pkg/api/api.swagger.go | 9 + pkg/api/api.swagger.json | 9 + pkg/api/event.pb.go | 392 ++++++----- pkg/api/event.proto | 2 + pkg/api/submit.pb.go | 437 +++++++------ pkg/api/submit.proto | 1 + pkg/armadaevents/events.pb.go | 607 +++++++++++------- pkg/armadaevents/events.proto | 3 + 26 files changed, 1178 insertions(+), 598 deletions(-) create mode 100644 internal/scheduler/scheduling/preemption_description.go create mode 100644 internal/scheduler/scheduling/preemption_description_test.go diff --git a/cmd/armadactl/cmd/preempt.go b/cmd/armadactl/cmd/preempt.go index 8380f691d75..4a7adc250d3 100644 --- a/cmd/armadactl/cmd/preempt.go +++ b/cmd/armadactl/cmd/preempt.go @@ -24,10 +24,10 @@ func preemptCmd() *cobra.Command { func preemptJobCmd() *cobra.Command { a := armadactl.New() cmd := &cobra.Command{ - Use: "job ", + Use: "job ", Short: "Preempt an armada job.", - Long: `Preempt a job by providing it's queue, jobset and jobId.`, - Args: cobra.ExactArgs(3), + Long: `Preempt a job by providing it's queue, jobset, jobId and a preemption reason.`, + Args: cobra.ExactArgs(4), PreRunE: func(cmd *cobra.Command, args []string) error { return initParams(cmd, a.Params) }, @@ -35,7 +35,8 @@ func preemptJobCmd() *cobra.Command { queue := args[0] jobSetId := args[1] jobId := args[2] - return a.Preempt(queue, jobSetId, jobId) + reason := args[3] + return a.Preempt(queue, jobSetId, jobId, reason) }, } return cmd diff --git a/internal/armadactl/preempt.go b/internal/armadactl/preempt.go index 185f7b8e70f..a49fbdd6590 100644 --- a/internal/armadactl/preempt.go +++ b/internal/armadactl/preempt.go @@ -13,7 +13,7 @@ import ( ) // Preempt a job. -func (a *App) Preempt(queue string, jobSetId string, jobId string) (outerErr error) { +func (a *App) Preempt(queue string, jobSetId string, jobId string, reason string) (outerErr error) { apiConnectionDetails := a.Params.ApiConnectionDetails fmt.Fprintf(a.Out, "Requesting preemption of job matching queue: %s, job set: %s, and job Id: %s\n", queue, jobSetId, jobId) @@ -25,6 +25,7 @@ func (a *App) Preempt(queue string, jobSetId string, jobId string) (outerErr err JobIds: []string{jobId}, JobSetId: jobSetId, Queue: queue, + Reason: reason, }) if err != nil { return errors.Wrapf(err, "error preempting job matching queue: %s, job set: %s, and job id: %s", queue, jobSetId, jobId) diff --git a/internal/common/ingest/testfixtures/event.go b/internal/common/ingest/testfixtures/event.go index 05a38e665aa..e02399982b5 100644 --- a/internal/common/ingest/testfixtures/event.go +++ b/internal/common/ingest/testfixtures/event.go @@ -39,6 +39,7 @@ const ( DebugMsg = "sample debug message" LeaseReturnedMsg = "lease returned error message" UnschedulableMsg = "test pod is unschedulable" + PreemptionReason = "job preempted" PartitionMarkerPartitionId = 456 ExecutorCordonReason = "bad executor" @@ -367,6 +368,7 @@ var JobRunPreempted = &armadaevents.EventSequence_Event{ JobRunPreempted: &armadaevents.JobRunPreempted{ PreemptedJobId: JobId, PreemptedRunId: RunId, + Reason: PreemptionReason, }, }, } diff --git a/internal/lookoutingesterv2/instructions/instructions.go b/internal/lookoutingesterv2/instructions/instructions.go index f5ab3742d75..6be893b85ea 100644 --- a/internal/lookoutingesterv2/instructions/instructions.go +++ b/internal/lookoutingesterv2/instructions/instructions.go @@ -431,7 +431,7 @@ func (c *InstructionConverter) handleJobRunPreempted(ts time.Time, event *armada RunId: event.PreemptedRunId, JobRunState: pointer.Int32(lookout.JobRunPreemptedOrdinal), Finished: &ts, - Error: tryCompressError(event.PreemptedJobId, "preempted", c.compressor), + Error: tryCompressError(event.PreemptedJobId, event.Reason, c.compressor), } update.JobRunsToUpdate = append(update.JobRunsToUpdate, &jobRun) return nil diff --git a/internal/lookoutingesterv2/instructions/instructions_test.go b/internal/lookoutingesterv2/instructions/instructions_test.go index 210f706f16e..19dee86f1b9 100644 --- a/internal/lookoutingesterv2/instructions/instructions_test.go +++ b/internal/lookoutingesterv2/instructions/instructions_test.go @@ -160,7 +160,7 @@ var expectedPreemptedRun = model.UpdateJobRunInstruction{ RunId: testfixtures.RunId, Finished: &testfixtures.BaseTime, JobRunState: pointer.Int32(lookout.JobRunPreemptedOrdinal), - Error: []byte("preempted"), + Error: []byte(testfixtures.PreemptionReason), } var expectedCancelledRun = model.UpdateJobRunInstruction{ diff --git a/internal/scheduler/nodedb/nodedb.go b/internal/scheduler/nodedb/nodedb.go index b2ab1676d6d..7d257de5a4e 100644 --- a/internal/scheduler/nodedb/nodedb.go +++ b/internal/scheduler/nodedb/nodedb.go @@ -341,6 +341,7 @@ func (nodeDb *NodeDb) ScheduleManyWithTxn(txn *memdb.Txn, gctx *context.GangSche // order to find the best fit for this gang); clear out any remnants of // previous attempts. jctx.UnschedulableReason = "" + jctx.PreemptingJobId = "" node, err := nodeDb.SelectNodeForJobWithTxn(txn, jctx) if err != nil { @@ -420,6 +421,7 @@ func (nodeDb *NodeDb) SelectNodeForJobWithTxn(txn *memdb.Txn, jctx *context.JobS if node, err := nodeDb.selectNodeForPodWithItAtPriority(it, jctx, priority, true); err != nil { return nil, err } else { + jctx.PodSchedulingContext.SchedulingMethod = context.Rescheduled return node, nil } } @@ -440,6 +442,7 @@ func (nodeDb *NodeDb) SelectNodeForJobWithTxn(txn *memdb.Txn, jctx *context.JobS } if node != nil { pctx.WellKnownNodeTypeName = awayNodeType.WellKnownNodeTypeName + pctx.SchedulingMethod = context.ScheduledAsAwayJob pctx.ScheduledAway = true return node, nil } @@ -499,6 +502,7 @@ func (nodeDb *NodeDb) selectNodeForJobWithTxnAtPriority( } else if err := assertPodSchedulingContextNode(pctx, node); err != nil { return nil, err } else if node != nil { + pctx.SchedulingMethod = context.ScheduledWithoutPreemption return node, nil } @@ -522,6 +526,7 @@ func (nodeDb *NodeDb) selectNodeForJobWithTxnAtPriority( } else if err := assertPodSchedulingContextNode(pctx, node); err != nil { return nil, err } else if node != nil { + pctx.SchedulingMethod = context.ScheduledWithFairSharePreemption return node, nil } @@ -535,6 +540,7 @@ func (nodeDb *NodeDb) selectNodeForJobWithTxnAtPriority( } else if err := assertPodSchedulingContextNode(pctx, node); err != nil { return nil, err } else if node != nil { + pctx.SchedulingMethod = context.ScheduledWithUrgencyBasedPreemption return node, nil } @@ -760,6 +766,7 @@ func (nodeDb *NodeDb) selectNodeForJobWithFairPreemption(txn *memdb.Txn, jctx *c if priority > maxPriority { maxPriority = priority } + job.JobSchedulingContext.PreemptingJobId = jctx.JobId } selectedNode = nodeCopy diff --git a/internal/scheduler/scheduler.go b/internal/scheduler/scheduler.go index e5f86f495f5..fd796e1bd82 100644 --- a/internal/scheduler/scheduler.go +++ b/internal/scheduler/scheduler.go @@ -470,7 +470,7 @@ func (s *Scheduler) eventsFromSchedulerResult(result *scheduling.SchedulerResult // EventsFromSchedulerResult generates necessary EventSequences from the provided SchedulerResult. func EventsFromSchedulerResult(result *scheduling.SchedulerResult, time time.Time) ([]*armadaevents.EventSequence, error) { eventSequences := make([]*armadaevents.EventSequence, 0, len(result.PreemptedJobs)+len(result.ScheduledJobs)) - eventSequences, err := AppendEventSequencesFromPreemptedJobs(eventSequences, scheduling.PreemptedJobsFromSchedulerResult(result), time) + eventSequences, err := AppendEventSequencesFromPreemptedJobs(eventSequences, result.PreemptedJobs, time) if err != nil { return nil, err } @@ -481,22 +481,22 @@ func EventsFromSchedulerResult(result *scheduling.SchedulerResult, time time.Tim return eventSequences, nil } -func AppendEventSequencesFromPreemptedJobs(eventSequences []*armadaevents.EventSequence, jobs []*jobdb.Job, time time.Time) ([]*armadaevents.EventSequence, error) { - for _, job := range jobs { - run := job.LatestRun() +func AppendEventSequencesFromPreemptedJobs(eventSequences []*armadaevents.EventSequence, jctxs []*schedulercontext.JobSchedulingContext, time time.Time) ([]*armadaevents.EventSequence, error) { + for _, jctx := range jctxs { + run := jctx.Job.LatestRun() if run == nil { - return nil, errors.Errorf("attempting to generate preempted eventSequences for job %s with no associated runs", job.Id()) + return nil, errors.Errorf("attempting to generate preempted eventSequences for job %s with no associated runs", jctx.JobId) } eventSequences = append(eventSequences, &armadaevents.EventSequence{ - Queue: job.Queue(), - JobSetName: job.Jobset(), - Events: createEventsForPreemptedJob(job.Id(), run.Id(), time), + Queue: jctx.Job.Queue(), + JobSetName: jctx.Job.Jobset(), + Events: createEventsForPreemptedJob(jctx.JobId, run.Id(), jctx.PreemptionDescription, time), }) } return eventSequences, nil } -func createEventsForPreemptedJob(jobId string, runId string, time time.Time) []*armadaevents.EventSequence_Event { +func createEventsForPreemptedJob(jobId string, runId string, reason string, time time.Time) []*armadaevents.EventSequence_Event { return []*armadaevents.EventSequence_Event{ { Created: protoutil.ToTimestamp(time), @@ -504,6 +504,7 @@ func createEventsForPreemptedJob(jobId string, runId string, time time.Time) []* JobRunPreempted: &armadaevents.JobRunPreempted{ PreemptedRunId: runId, PreemptedJobId: jobId, + Reason: reason, }, }, }, @@ -517,7 +518,9 @@ func createEventsForPreemptedJob(jobId string, runId string, time time.Time) []* { Terminal: true, Reason: &armadaevents.Error_JobRunPreemptedError{ - JobRunPreemptedError: &armadaevents.JobRunPreemptedError{}, + JobRunPreemptedError: &armadaevents.JobRunPreemptedError{ + Reason: reason, + }, }, }, }, @@ -533,7 +536,9 @@ func createEventsForPreemptedJob(jobId string, runId string, time time.Time) []* { Terminal: true, Reason: &armadaevents.Error_JobRunPreemptedError{ - JobRunPreemptedError: &armadaevents.JobRunPreemptedError{}, + JobRunPreemptedError: &armadaevents.JobRunPreemptedError{ + Reason: reason, + }, }, }, }, @@ -776,7 +781,7 @@ func (s *Scheduler) generateUpdateMessagesFromJob(ctx *armadacontext.Context, jo } } else if lastRun.PreemptRequested() && job.PriorityClass().Preemptible { job = job.WithQueued(false).WithFailed(true).WithUpdatedRun(lastRun.WithoutTerminal().WithFailed(true)) - events = append(events, createEventsForPreemptedJob(job.Id(), lastRun.Id(), s.clock.Now())...) + events = append(events, createEventsForPreemptedJob(job.Id(), lastRun.Id(), "Preempted - preemption requested via API", s.clock.Now())...) } } diff --git a/internal/scheduler/scheduling/context/job.go b/internal/scheduler/scheduling/context/job.go index ae8fd0966dd..2b96b335f1b 100644 --- a/internal/scheduler/scheduling/context/job.go +++ b/internal/scheduler/scheduling/context/job.go @@ -60,6 +60,10 @@ type JobSchedulingContext struct { // This is the node the pod is assigned to. // This is only set for evicted jobs and is set alongside adding an additionalNodeSelector for the node AssignedNodeId string + // Id of job that preempted this pod + PreemptingJobId string + // Description of the cause of preemption + PreemptionDescription string } func (jctx *JobSchedulingContext) IsHomeJob(currentPool string) bool { @@ -101,6 +105,7 @@ func (jctx *JobSchedulingContext) Fail(unschedulableReason string) { jctx.UnschedulableReason = unschedulableReason if pctx := jctx.PodSchedulingContext; pctx != nil { pctx.NodeId = "" + pctx.SchedulingMethod = None } } diff --git a/internal/scheduler/scheduling/context/pod.go b/internal/scheduler/scheduling/context/pod.go index 1d0b2595cb8..38ac730bf2d 100644 --- a/internal/scheduler/scheduling/context/pod.go +++ b/internal/scheduler/scheduling/context/pod.go @@ -7,6 +7,17 @@ import ( "time" ) +type SchedulingType int + +const ( + None SchedulingType = iota + Rescheduled + ScheduledWithoutPreemption + ScheduledWithFairSharePreemption + ScheduledWithUrgencyBasedPreemption + ScheduledAsAwayJob +) + // PodSchedulingContext is returned by SelectAndBindNodeToPod and // contains detailed information on the scheduling decision made for this pod. type PodSchedulingContext struct { @@ -27,6 +38,8 @@ type PodSchedulingContext struct { NumExcludedNodesByReason map[string]int // If this pod was scheduled as an away job ScheduledAway bool + // The method of scheduling that was used to schedule this job + SchedulingMethod SchedulingType } func (pctx *PodSchedulingContext) IsSuccessful() bool { diff --git a/internal/scheduler/scheduling/preempting_queue_scheduler.go b/internal/scheduler/scheduling/preempting_queue_scheduler.go index 428db27ff9f..34d9d36ae9f 100644 --- a/internal/scheduler/scheduling/preempting_queue_scheduler.go +++ b/internal/scheduler/scheduling/preempting_queue_scheduler.go @@ -230,6 +230,7 @@ func (sch *PreemptingQueueScheduler) Schedule(ctx *armadacontext.Context) (*Sche } ctx.WithField("stage", "scheduling-algo").Infof("Finished unbinding preempted and evicted jobs") + PopulatePreemptionDescriptions(preemptedJobs, scheduledJobs) schedulercontext.PrintJobSummary(ctx, "Preempting running jobs;", preemptedJobs) schedulercontext.PrintJobSummary(ctx, "Scheduling new jobs;", scheduledJobs) // TODO: Show failed jobs. diff --git a/internal/scheduler/scheduling/preemption_description.go b/internal/scheduler/scheduling/preemption_description.go new file mode 100644 index 00000000000..3ae3e595e38 --- /dev/null +++ b/internal/scheduler/scheduling/preemption_description.go @@ -0,0 +1,60 @@ +package scheduling + +import ( + "fmt" + "strings" + + armadaslices "github.com/armadaproject/armada/internal/common/slices" + "github.com/armadaproject/armada/internal/scheduler/scheduling/context" + "github.com/armadaproject/armada/internal/server/configuration" +) + +const ( + unknownPreemptionCause = "Preempted by scheduler due to the job failing to reschedule - possibly node resource changed causing this job to be unschedulable" + unknownGangPreemptionCause = "Preempted by scheduler due to the job failing to reschedule - possibly another job in the gang was preempted or the node resource changed causing this job to be unschedulable" + fairSharePreemptionTemplate = "Preempted by scheduler using fair share preemption - preempting job %s" + urgencyPreemptionTemplate = "Preempted by scheduler using urgency preemption - preempting job %s" + urgencyPreemptionMultiJobTemplate = "Preempted by scheduler using urgency preemption - preemption caused by one of the following jobs %s" +) + +func PopulatePreemptionDescriptions(preemptedJobs []*context.JobSchedulingContext, scheduledJobs []*context.JobSchedulingContext) { + jobsScheduledWithUrgencyBasedPreemptionByNode := map[string][]*context.JobSchedulingContext{} + for _, schedJctx := range scheduledJobs { + if schedJctx.PodSchedulingContext == nil { + continue + } + if schedJctx.PodSchedulingContext.SchedulingMethod != context.ScheduledWithUrgencyBasedPreemption { + continue + } + + nodeId := schedJctx.PodSchedulingContext.NodeId + if _, ok := jobsScheduledWithUrgencyBasedPreemptionByNode[nodeId]; !ok { + jobsScheduledWithUrgencyBasedPreemptionByNode[nodeId] = []*context.JobSchedulingContext{} + } + jobsScheduledWithUrgencyBasedPreemptionByNode[nodeId] = append(jobsScheduledWithUrgencyBasedPreemptionByNode[nodeId], schedJctx) + } + + for _, preemptedJctx := range preemptedJobs { + if preemptedJctx.PreemptingJobId != "" { + preemptedJctx.PreemptionDescription = fmt.Sprintf(fairSharePreemptionTemplate, preemptedJctx.PreemptingJobId) + } else { + potentialPreemptingJobs := jobsScheduledWithUrgencyBasedPreemptionByNode[preemptedJctx.GetAssignedNodeId()] + + if len(potentialPreemptingJobs) == 0 { + _, isGang := preemptedJctx.Job.Annotations()[configuration.GangIdAnnotation] + if isGang { + preemptedJctx.PreemptionDescription = fmt.Sprintf(unknownGangPreemptionCause) + } else { + preemptedJctx.PreemptionDescription = fmt.Sprintf(unknownPreemptionCause) + } + } else if len(potentialPreemptingJobs) == 1 { + preemptedJctx.PreemptionDescription = fmt.Sprintf(urgencyPreemptionTemplate, potentialPreemptingJobs[0].JobId) + } else { + jobIds := armadaslices.Map(potentialPreemptingJobs, func(job *context.JobSchedulingContext) string { + return job.JobId + }) + preemptedJctx.PreemptionDescription = fmt.Sprintf(urgencyPreemptionMultiJobTemplate, strings.Join(jobIds, ",")) + } + } + } +} diff --git a/internal/scheduler/scheduling/preemption_description_test.go b/internal/scheduler/scheduling/preemption_description_test.go new file mode 100644 index 00000000000..0d70f3dad0a --- /dev/null +++ b/internal/scheduler/scheduling/preemption_description_test.go @@ -0,0 +1,141 @@ +package scheduling + +import ( + "fmt" + "testing" + + "github.com/stretchr/testify/assert" + "github.com/stretchr/testify/require" + + "github.com/armadaproject/armada/internal/scheduler/jobdb" + "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/scheduling/context" + "github.com/armadaproject/armada/internal/scheduler/testfixtures" + "github.com/armadaproject/armada/internal/server/configuration" +) + +func TestPopulatePreemptionDescriptions(t *testing.T) { + scheduledJobContexts := []*context.JobSchedulingContext{ + makeJobSchedulingContext("job-2", "node-1", context.ScheduledWithUrgencyBasedPreemption), + makeJobSchedulingContext("job-3", "node-2", context.ScheduledWithUrgencyBasedPreemption), + makeJobSchedulingContext("job-4", "node-2", context.ScheduledWithUrgencyBasedPreemption), + makeJobSchedulingContext("job-5", "node-1", context.ScheduledWithFairSharePreemption), + makeJobSchedulingContext("job-5", "node-2", context.ScheduledWithFairSharePreemption), + makeJobSchedulingContext("job-6", "node-3", context.ScheduledWithFairSharePreemption), + } + expectedScheduleJobContexts := []*context.JobSchedulingContext{ + makeJobSchedulingContext("job-2", "node-1", context.ScheduledWithUrgencyBasedPreemption), + makeJobSchedulingContext("job-3", "node-2", context.ScheduledWithUrgencyBasedPreemption), + makeJobSchedulingContext("job-4", "node-2", context.ScheduledWithUrgencyBasedPreemption), + makeJobSchedulingContext("job-5", "node-1", context.ScheduledWithFairSharePreemption), + makeJobSchedulingContext("job-5", "node-2", context.ScheduledWithFairSharePreemption), + makeJobSchedulingContext("job-6", "node-3", context.ScheduledWithFairSharePreemption), + } + + tests := map[string]struct { + preemptedJobContext *context.JobSchedulingContext + expectedPreemptedJobContext *context.JobSchedulingContext + }{ + "unknown cause - basic job": { + preemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-3", + Job: makeJob(t, "job-1", false), + }, + expectedPreemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-3", + Job: makeJob(t, "job-1", false), + PreemptionDescription: unknownPreemptionCause, + }, + }, + "unknown cause - gang job": { + preemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-3", + Job: makeJob(t, "job-1", true), + }, + expectedPreemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-3", + Job: makeJob(t, "job-1", true), + PreemptionDescription: unknownGangPreemptionCause, + }, + }, + "urgency preemption - single preempting job": { + preemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-1", + }, + expectedPreemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-1", + PreemptionDescription: fmt.Sprintf(urgencyPreemptionTemplate, "job-2"), + }, + }, + "urgency preemption - multiple preempting jobs": { + preemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-2", + }, + expectedPreemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-2", + PreemptionDescription: fmt.Sprintf(urgencyPreemptionMultiJobTemplate, "job-3,job-4"), + }, + }, + "fairshare": { + preemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-4", + PreemptingJobId: "job-7", + }, + expectedPreemptedJobContext: &context.JobSchedulingContext{ + JobId: "job-1", + AssignedNodeId: "node-4", + PreemptingJobId: "job-7", + PreemptionDescription: fmt.Sprintf(fairSharePreemptionTemplate, "job-7"), + }, + }, + } + + for name, tc := range tests { + t.Run(name, func(t *testing.T) { + PopulatePreemptionDescriptions([]*context.JobSchedulingContext{tc.preemptedJobContext}, scheduledJobContexts) + assert.Equal(t, expectedScheduleJobContexts, scheduledJobContexts) + assert.Equal(t, []*context.JobSchedulingContext{tc.expectedPreemptedJobContext}, []*context.JobSchedulingContext{tc.preemptedJobContext}) + }) + } +} + +func makeJobSchedulingContext(jobId string, nodeId string, schedulingMethod context.SchedulingType) *context.JobSchedulingContext { + return &context.JobSchedulingContext{ + PodSchedulingContext: &context.PodSchedulingContext{ + SchedulingMethod: schedulingMethod, + NodeId: nodeId, + }, + JobId: jobId, + } +} + +func makeJob(t *testing.T, jobId string, isGang bool) *jobdb.Job { + annotations := map[string]string{} + if isGang { + annotations[configuration.GangIdAnnotation] = "gang" + } + schedulingInfo := &schedulerobjects.JobSchedulingInfo{ + ObjectRequirements: []*schedulerobjects.ObjectRequirements{ + { + Requirements: &schedulerobjects.ObjectRequirements_PodRequirements{ + PodRequirements: &schedulerobjects.PodRequirements{ + Annotations: annotations, + }, + }, + }, + }, + } + + job, err := testfixtures.JobDb.NewJob(jobId, "jobset", "queue", 1, schedulingInfo, false, 1, false, false, false, 0, true, []string{}) + require.NoError(t, err) + return job +} diff --git a/internal/scheduler/simulator/simulator.go b/internal/scheduler/simulator/simulator.go index b1f5a7e1838..aec3a3d55c2 100644 --- a/internal/scheduler/simulator/simulator.go +++ b/internal/scheduler/simulator/simulator.go @@ -623,8 +623,9 @@ func (s *Simulator) handleScheduleEvent(ctx *armadacontext.Context) error { // Update jobDb to reflect the decisions by the scheduler. // Sort jobs to ensure deterministic event ordering. - preemptedJobs := scheduling.PreemptedJobsFromSchedulerResult(result) + preemptedJobs := slices.Clone(result.PreemptedJobs) scheduledJobs := slices.Clone(result.ScheduledJobs) + lessJob := func(a, b *jobdb.Job) int { if a.Queue() < b.Queue() { return -1 @@ -638,18 +639,21 @@ func (s *Simulator) handleScheduleEvent(ctx *armadacontext.Context) error { } return 0 } - slices.SortFunc(preemptedJobs, lessJob) + slices.SortFunc(preemptedJobs, func(a, b *schedulercontext.JobSchedulingContext) int { + return lessJob(a.Job, b.Job) + }) slices.SortFunc(scheduledJobs, func(a, b *schedulercontext.JobSchedulingContext) int { return lessJob(a.Job, b.Job) }) - for i, job := range preemptedJobs { + for i, jctx := range preemptedJobs { + job := jctx.Job delete(s.accounting.nodeIdByJobId, job.Id()) if run := job.LatestRun(); run != nil { job = job.WithUpdatedRun(run.WithFailed(true)) } else { return errors.Errorf("attempting to preempt job %s with no associated runs", job.Id()) } - preemptedJobs[i] = job.WithQueued(false).WithFailed(true) + preemptedJobs[i].Job = job.WithQueued(false).WithFailed(true) } for i, jctx := range scheduledJobs { job := jctx.Job @@ -668,7 +672,7 @@ func (s *Simulator) handleScheduleEvent(ctx *armadacontext.Context) error { scheduledJobs[i].Job = job.WithQueued(false).WithNewRun(node.GetExecutor(), node.GetId(), node.GetName(), node.GetPool(), priority) } } - if err := txn.Upsert(preemptedJobs); err != nil { + if err := txn.Upsert(armadaslices.Map(preemptedJobs, func(jctx *schedulercontext.JobSchedulingContext) *jobdb.Job { return jctx.Job })); err != nil { return err } if err := txn.Upsert(armadaslices.Map(scheduledJobs, func(jctx *schedulercontext.JobSchedulingContext) *jobdb.Job { return jctx.Job })); err != nil { @@ -679,7 +683,7 @@ func (s *Simulator) handleScheduleEvent(ctx *armadacontext.Context) error { s.accounting.allocationByPoolAndQueueAndPriorityClass[pool] = sctx.AllocatedByQueueAndPriority() // Generate eventSequences. - eventSequences, err = scheduler.AppendEventSequencesFromPreemptedJobs(eventSequences, preemptedJobs, s.time) + eventSequences, err = scheduler.AppendEventSequencesFromPreemptedJobs(eventSequences, result.PreemptedJobs, s.time) if err != nil { return err } diff --git a/internal/server/event/conversion/conversions.go b/internal/server/event/conversion/conversions.go index fb4fa69d8cf..227f9084a93 100644 --- a/internal/server/event/conversion/conversions.go +++ b/internal/server/event/conversion/conversions.go @@ -119,6 +119,7 @@ func FromInternalPreemptionRequested(userId string, queueName string, jobSetName Queue: queueName, Created: protoutil.ToTimestamp(time), Requestor: userId, + Reason: e.Reason, }, }, }, @@ -317,7 +318,7 @@ func FromInternalJobErrors(queueName string, jobSetName string, time time.Time, JobSetId: jobSetName, Queue: queueName, Created: protoutil.ToTimestamp(time), - Reason: "preempted", + Reason: reason.JobRunPreemptedError.Reason, }, }, } @@ -444,6 +445,7 @@ func FromInternalJobRunPreempted(queueName string, jobSetName string, time time. Queue: queueName, Created: protoutil.ToTimestamp(time), RunId: e.PreemptedRunId, + Reason: e.Reason, } return []*api.EventMessage{ diff --git a/internal/server/event/conversion/conversions_test.go b/internal/server/event/conversion/conversions_test.go index bbbfea9ddd3..95ae1ef6606 100644 --- a/internal/server/event/conversion/conversions_test.go +++ b/internal/server/event/conversion/conversions_test.go @@ -830,7 +830,8 @@ func TestConvertJobPreemptionRequested(t *testing.T) { Created: baseTimeProto, Event: &armadaevents.EventSequence_Event_JobPreemptionRequested{ JobPreemptionRequested: &armadaevents.JobPreemptionRequested{ - JobId: jobId, + JobId: jobId, + Reason: "preemption requested", }, }, } @@ -844,6 +845,7 @@ func TestConvertJobPreemptionRequested(t *testing.T) { Queue: queue, Created: protoutil.ToTimestamp(baseTime), Requestor: userId, + Reason: "preemption requested", }, }, }, @@ -861,6 +863,7 @@ func TestConvertJobRunPreempted(t *testing.T) { JobRunPreempted: &armadaevents.JobRunPreempted{ PreemptedJobId: jobId, PreemptedRunId: runId, + Reason: "Preempted reason", }, }, } @@ -874,6 +877,7 @@ func TestConvertJobRunPreempted(t *testing.T) { Queue: queue, Created: protoutil.ToTimestamp(baseTime), RunId: runId, + Reason: "Preempted reason", }, }, }, diff --git a/internal/server/submit/submit.go b/internal/server/submit/submit.go index 36db7b535cb..63ea39d1fa6 100644 --- a/internal/server/submit/submit.go +++ b/internal/server/submit/submit.go @@ -211,7 +211,7 @@ func (s *Server) PreemptJobs(grpcCtx context.Context, req *api.JobPreemptRequest return nil, err } - sequence, err := preemptJobEventSequenceForJobIds(s.clock, req.JobIds, req.Queue, req.JobSetId, userId, groups) + sequence, err := preemptJobEventSequenceForJobIds(s.clock, req.JobIds, req.Queue, req.JobSetId, userId, req.Reason, groups) if err != nil { return nil, err } @@ -225,7 +225,7 @@ func (s *Server) PreemptJobs(grpcCtx context.Context, req *api.JobPreemptRequest return &types.Empty{}, nil } -func preemptJobEventSequenceForJobIds(clock clock.Clock, jobIds []string, q, jobSet, userId string, groups []string) (*armadaevents.EventSequence, error) { +func preemptJobEventSequenceForJobIds(clock clock.Clock, jobIds []string, q, jobSet, userId, reason string, groups []string) (*armadaevents.EventSequence, error) { sequence := &armadaevents.EventSequence{ Queue: q, JobSetName: jobSet, @@ -239,7 +239,8 @@ func preemptJobEventSequenceForJobIds(clock clock.Clock, jobIds []string, q, job Created: eventTime, Event: &armadaevents.EventSequence_Event_JobPreemptionRequested{ JobPreemptionRequested: &armadaevents.JobPreemptionRequested{ - JobId: jobId, + JobId: jobId, + Reason: reason, }, }, }) diff --git a/internal/server/submit/submit_test.go b/internal/server/submit/submit_test.go index 5edacdf0c69..3a67b2e7209 100644 --- a/internal/server/submit/submit_test.go +++ b/internal/server/submit/submit_test.go @@ -294,8 +294,8 @@ func TestPreemptJobs(t *testing.T) { expectedEvents []*armadaevents.EventSequence_Event }{ "Preempt jobs using JobIds": { - req: &api.JobPreemptRequest{JobIds: []string{jobId1, jobId2}, Queue: testfixtures.DefaultQueue.Name, JobSetId: testfixtures.DefaultJobset}, - expectedEvents: testfixtures.CreatePreemptJobSequenceEvents([]string{jobId1, jobId2}), + req: &api.JobPreemptRequest{JobIds: []string{jobId1, jobId2}, Queue: testfixtures.DefaultQueue.Name, JobSetId: testfixtures.DefaultJobset, Reason: "preemption-reason"}, + expectedEvents: testfixtures.CreatePreemptJobSequenceEvents([]string{jobId1, jobId2}, "preemption-reason"), }, } for name, tc := range tests { diff --git a/internal/server/submit/testfixtures/test_fixtures.go b/internal/server/submit/testfixtures/test_fixtures.go index ae2ce7ff859..75d1595d9e2 100644 --- a/internal/server/submit/testfixtures/test_fixtures.go +++ b/internal/server/submit/testfixtures/test_fixtures.go @@ -74,14 +74,15 @@ func DefaultSubmissionConfig() configuration.SubmissionConfig { } } -func CreatePreemptJobSequenceEvents(jobIds []string) []*armadaevents.EventSequence_Event { +func CreatePreemptJobSequenceEvents(jobIds []string, reason string) []*armadaevents.EventSequence_Event { events := make([]*armadaevents.EventSequence_Event, len(jobIds)) for i, jobId := range jobIds { events[i] = &armadaevents.EventSequence_Event{ Created: DefaultTimeProto, Event: &armadaevents.EventSequence_Event_JobPreemptionRequested{ JobPreemptionRequested: &armadaevents.JobPreemptionRequested{ - JobId: jobId, + JobId: jobId, + Reason: reason, }, }, } diff --git a/pkg/api/api.swagger.go b/pkg/api/api.swagger.go index e8b09083a49..239c355c8fa 100644 --- a/pkg/api/api.swagger.go +++ b/pkg/api/api.swagger.go @@ -1323,6 +1323,9 @@ func SwaggerJsonTemplate() string { " },\n" + " \"queue\": {\n" + " \"type\": \"string\"\n" + + " },\n" + + " \"reason\": {\n" + + " \"type\": \"string\"\n" + " }\n" + " }\n" + " },\n" + @@ -1351,6 +1354,9 @@ func SwaggerJsonTemplate() string { " \"queue\": {\n" + " \"type\": \"string\"\n" + " },\n" + + " \"reason\": {\n" + + " \"type\": \"string\"\n" + + " },\n" + " \"runId\": {\n" + " \"type\": \"string\"\n" + " }\n" + @@ -1372,6 +1378,9 @@ func SwaggerJsonTemplate() string { " \"queue\": {\n" + " \"type\": \"string\"\n" + " },\n" + + " \"reason\": {\n" + + " \"type\": \"string\"\n" + + " },\n" + " \"requestor\": {\n" + " \"type\": \"string\"\n" + " }\n" + diff --git a/pkg/api/api.swagger.json b/pkg/api/api.swagger.json index 8053aefeddd..f56ab357ce5 100644 --- a/pkg/api/api.swagger.json +++ b/pkg/api/api.swagger.json @@ -1312,6 +1312,9 @@ }, "queue": { "type": "string" + }, + "reason": { + "type": "string" } } }, @@ -1340,6 +1343,9 @@ "queue": { "type": "string" }, + "reason": { + "type": "string" + }, "runId": { "type": "string" } @@ -1361,6 +1367,9 @@ "queue": { "type": "string" }, + "reason": { + "type": "string" + }, "requestor": { "type": "string" } diff --git a/pkg/api/event.pb.go b/pkg/api/event.pb.go index 777e35ac34c..df8053c7add 100644 --- a/pkg/api/event.pb.go +++ b/pkg/api/event.pb.go @@ -1088,6 +1088,7 @@ type JobPreemptingEvent struct { Queue string `protobuf:"bytes,3,opt,name=queue,proto3" json:"queue,omitempty"` Created *types.Timestamp `protobuf:"bytes,4,opt,name=created,proto3" json:"created,omitempty"` Requestor string `protobuf:"bytes,5,opt,name=requestor,proto3" json:"requestor,omitempty"` + Reason string `protobuf:"bytes,6,opt,name=reason,proto3" json:"reason,omitempty"` } func (m *JobPreemptingEvent) Reset() { *m = JobPreemptingEvent{} } @@ -1158,6 +1159,13 @@ func (m *JobPreemptingEvent) GetRequestor() string { return "" } +func (m *JobPreemptingEvent) GetReason() string { + if m != nil { + return m.Reason + } + return "" +} + type JobPreemptedEvent struct { JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId,proto3" json:"jobId,omitempty"` JobSetId string `protobuf:"bytes,2,opt,name=job_set_id,json=jobSetId,proto3" json:"jobSetId,omitempty"` @@ -1167,6 +1175,7 @@ type JobPreemptedEvent struct { RunId string `protobuf:"bytes,6,opt,name=run_id,json=runId,proto3" json:"runId,omitempty"` PreemptiveJobId string `protobuf:"bytes,7,opt,name=preemptive_job_id,json=preemptiveJobId,proto3" json:"preemptiveJobId,omitempty"` PreemptiveRunId string `protobuf:"bytes,8,opt,name=preemptive_run_id,json=preemptiveRunId,proto3" json:"preemptiveRunId,omitempty"` + Reason string `protobuf:"bytes,9,opt,name=reason,proto3" json:"reason,omitempty"` } func (m *JobPreemptedEvent) Reset() { *m = JobPreemptedEvent{} } @@ -1258,6 +1267,13 @@ func (m *JobPreemptedEvent) GetPreemptiveRunId() string { return "" } +func (m *JobPreemptedEvent) GetReason() string { + if m != nil { + return m.Reason + } + return "" +} + type JobSucceededEvent struct { JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId,proto3" json:"jobId,omitempty"` JobSetId string `protobuf:"bytes,2,opt,name=job_set_id,json=jobSetId,proto3" json:"jobSetId,omitempty"` @@ -2552,151 +2568,151 @@ func init() { func init() { proto.RegisterFile("pkg/api/event.proto", fileDescriptor_7758595c3bb8cf56) } var fileDescriptor_7758595c3bb8cf56 = []byte{ - // 2290 bytes of a gzipped FileDescriptorProto - 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xec, 0x5b, 0xcd, 0x6f, 0x1c, 0xb7, - 0x15, 0xf7, 0xac, 0xb4, 0x5f, 0x5c, 0x7d, 0xd2, 0x92, 0x3c, 0x5e, 0xc7, 0x1a, 0x61, 0x0c, 0xb4, - 0x8a, 0x11, 0xef, 0xa6, 0x72, 0x52, 0x04, 0x46, 0x81, 0xc2, 0xab, 0x28, 0x89, 0xd4, 0xb8, 0x76, - 0x56, 0x36, 0x92, 0x16, 0x01, 0xb6, 0xf3, 0x41, 0x49, 0x23, 0xed, 0x0e, 0x27, 0xf3, 0x61, 0x5b, - 0x31, 0x72, 0x69, 0x2f, 0x3d, 0x14, 0x4d, 0x3f, 0xae, 0x45, 0x5b, 0x14, 0xe8, 0xa5, 0x40, 0x2f, - 0xfd, 0x0b, 0x7a, 0x28, 0x8a, 0x1e, 0x83, 0xe4, 0xd2, 0xd3, 0xa2, 0xb0, 0x8b, 0x04, 0xd8, 0x53, - 0xef, 0xbd, 0x14, 0x7c, 0xe4, 0xcc, 0x92, 0xe3, 0x15, 0x2c, 0x2b, 0x4d, 0x62, 0x28, 0x7b, 0xb2, - 0xf5, 0x7b, 0x7c, 0x8f, 0x6f, 0x1e, 0x7f, 0x24, 0x1f, 0xf9, 0xb8, 0xe8, 0x6c, 0x70, 0xb0, 0xdb, - 0xb4, 0x02, 0xaf, 0x49, 0xee, 0x12, 0x3f, 0x6e, 0x04, 0x21, 0x8d, 0x29, 0x9e, 0xb0, 0x02, 0xaf, - 0x6e, 0xec, 0x52, 0xba, 0xdb, 0x25, 0x4d, 0x80, 0xec, 0x64, 0xa7, 0x19, 0x7b, 0x3d, 0x12, 0xc5, - 0x56, 0x2f, 0xe0, 0xad, 0xea, 0x0b, 0xa9, 0x6a, 0x94, 0xd8, 0x3d, 0x2f, 0xce, 0xa3, 0x7b, 0xc4, - 0xea, 0xc6, 0x7b, 0x02, 0xbd, 0x90, 0x37, 0x46, 0x7a, 0x41, 0x7c, 0x28, 0x84, 0xcf, 0x09, 0x21, - 0xd3, 0xb2, 0x7c, 0x9f, 0xc6, 0x56, 0xec, 0x51, 0x3f, 0x12, 0xd2, 0x97, 0x0e, 0x5e, 0x89, 0x1a, - 0x1e, 0x65, 0xd2, 0x9e, 0xe5, 0xec, 0x79, 0x3e, 0x09, 0x0f, 0x9b, 0x69, 0x27, 0x21, 0x89, 0x68, - 0x12, 0x3a, 0xa4, 0xb9, 0x4b, 0x7c, 0x12, 0x5a, 0x31, 0x71, 0xb9, 0x96, 0xf9, 0x9b, 0x02, 0x9a, - 0xdf, 0xa2, 0xf6, 0x36, 0xb8, 0x16, 0x13, 0x77, 0x83, 0x7d, 0x1e, 0xbe, 0x8c, 0x4a, 0xfb, 0xd4, - 0xee, 0x78, 0xae, 0xae, 0xad, 0x68, 0xab, 0xd5, 0xd6, 0xd9, 0x41, 0xdf, 0x98, 0xdd, 0xa7, 0xf6, - 0xa6, 0xfb, 0x02, 0xed, 0x79, 0x31, 0x38, 0xd5, 0x2e, 0x02, 0x80, 0x5f, 0x42, 0x88, 0xb5, 0x8d, - 0x48, 0xcc, 0xda, 0x17, 0xa0, 0xfd, 0xd2, 0xa0, 0x6f, 0xe0, 0x7d, 0x6a, 0x6f, 0x93, 0x58, 0x51, - 0xa9, 0xa4, 0x18, 0x7e, 0x1e, 0x15, 0xdf, 0x4b, 0x48, 0x42, 0xf4, 0x89, 0x61, 0x07, 0x00, 0xc8, - 0x1d, 0x00, 0x80, 0xbf, 0x87, 0xca, 0x4e, 0x48, 0x98, 0xcf, 0xfa, 0xe4, 0x8a, 0xb6, 0x5a, 0x5b, - 0xab, 0x37, 0x78, 0x20, 0x1a, 0x69, 0x94, 0x1a, 0xb7, 0xd3, 0x90, 0xb7, 0x16, 0x07, 0x7d, 0x63, - 0x5e, 0x34, 0x97, 0x4c, 0xa5, 0x16, 0xf0, 0x15, 0x34, 0xb1, 0x4f, 0x6d, 0xbd, 0x08, 0x86, 0x2a, - 0x0d, 0x2b, 0xf0, 0x1a, 0x5b, 0xd4, 0x6e, 0xcd, 0x0f, 0xfa, 0xc6, 0xf4, 0x3e, 0xb5, 0x25, 0x15, - 0xd6, 0xce, 0x1c, 0x68, 0x68, 0x66, 0x8b, 0xda, 0x6f, 0x31, 0x47, 0x4e, 0x7b, 0x6c, 0xcc, 0x3f, - 0x14, 0xe0, 0x63, 0xdf, 0x24, 0x56, 0x74, 0xfa, 0x89, 0xf0, 0x6d, 0x84, 0x9c, 0x6e, 0x12, 0xc5, - 0x24, 0x64, 0xde, 0x16, 0xa1, 0xf3, 0x73, 0x83, 0xbe, 0x71, 0x56, 0xa0, 0x8a, 0xbb, 0xd5, 0x0c, - 0x34, 0x7f, 0x39, 0x89, 0x16, 0xd3, 0x20, 0xb5, 0x49, 0x9c, 0x84, 0xfe, 0x38, 0x56, 0x47, 0xc4, - 0x0a, 0xbf, 0x80, 0x4a, 0x21, 0xb1, 0x22, 0xea, 0xeb, 0x25, 0xd0, 0x59, 0x18, 0xf4, 0x8d, 0x39, - 0x8e, 0x48, 0x0a, 0xa2, 0x0d, 0xfe, 0x2e, 0x9a, 0x3e, 0x48, 0x6c, 0x12, 0xfa, 0x24, 0x26, 0x11, - 0xeb, 0xa8, 0x0c, 0x4a, 0xf5, 0x41, 0xdf, 0x58, 0x1a, 0x0a, 0x94, 0xbe, 0xa6, 0x64, 0x9c, 0xb9, - 0x19, 0x50, 0xb7, 0xe3, 0x27, 0x3d, 0x9b, 0x84, 0x7a, 0x65, 0x45, 0x5b, 0x2d, 0x72, 0x37, 0x03, - 0xea, 0x7e, 0x1f, 0x40, 0xd9, 0xcd, 0x0c, 0x64, 0x1d, 0x87, 0x89, 0xdf, 0xb1, 0x62, 0x10, 0x11, - 0x57, 0xaf, 0xae, 0x68, 0xab, 0x15, 0xde, 0x71, 0x98, 0xf8, 0xd7, 0x53, 0x5c, 0xee, 0x58, 0xc6, - 0xcd, 0xff, 0x68, 0x68, 0x21, 0xe5, 0xc4, 0xc6, 0xfd, 0xc0, 0x0b, 0x4f, 0xff, 0x5a, 0xf1, 0xe1, - 0x24, 0x9a, 0xdd, 0xa2, 0xf6, 0x2d, 0xe2, 0xbb, 0x9e, 0xbf, 0x3b, 0x9e, 0x00, 0xa3, 0x27, 0xc0, - 0x63, 0x94, 0x2e, 0x7d, 0x2e, 0x4a, 0x97, 0x8f, 0x4d, 0xe9, 0x17, 0x51, 0x05, 0xf4, 0xac, 0x1e, - 0x81, 0x89, 0x50, 0xe5, 0x9f, 0xc8, 0x1a, 0x58, 0x3d, 0x39, 0x5a, 0x65, 0x01, 0x31, 0x57, 0x53, - 0x8d, 0x28, 0xb0, 0x1c, 0x02, 0x93, 0x40, 0xb8, 0x2a, 0xda, 0x00, 0x2e, 0xbb, 0x2a, 0xe3, 0xe6, - 0xdf, 0x38, 0x23, 0xda, 0x89, 0xef, 0x8f, 0x19, 0xf1, 0xc5, 0x31, 0xe2, 0x2a, 0xaa, 0xfa, 0xd4, - 0x25, 0x7c, 0x68, 0xcb, 0xc3, 0x28, 0x31, 0x30, 0x37, 0xb6, 0x95, 0x14, 0x3b, 0xf1, 0xca, 0x28, - 0xd3, 0xa8, 0x7a, 0x32, 0x1a, 0xa1, 0xa7, 0xa4, 0xd1, 0x5f, 0x4a, 0xe8, 0xec, 0x16, 0xb5, 0x37, - 0xfd, 0xdd, 0x90, 0x44, 0xd1, 0xa6, 0xbf, 0x43, 0xc7, 0x54, 0x3a, 0x6d, 0x54, 0x42, 0x27, 0xa3, - 0x52, 0xed, 0xe9, 0xa8, 0x84, 0x1f, 0xa0, 0x79, 0x8f, 0xd3, 0xa8, 0x63, 0xb9, 0x2e, 0xfb, 0x97, - 0x44, 0x7a, 0x75, 0x65, 0x62, 0xb5, 0xb6, 0xd6, 0x48, 0x33, 0xff, 0x3c, 0xcf, 0x1a, 0x02, 0xb8, - 0x9e, 0x2a, 0x6c, 0xf8, 0x71, 0x78, 0xd8, 0x5a, 0x1e, 0xf4, 0x8d, 0xba, 0x97, 0x13, 0x49, 0x1d, - 0xcf, 0xe5, 0x65, 0xf5, 0x03, 0xb4, 0x38, 0xd2, 0x14, 0xbe, 0x84, 0x26, 0x0e, 0xc8, 0x21, 0xb0, - 0xb8, 0xc8, 0xcf, 0x1d, 0x07, 0xe4, 0x50, 0x3e, 0x77, 0x1c, 0x90, 0x43, 0xc6, 0xc5, 0xbb, 0x56, - 0x37, 0x21, 0x82, 0xbc, 0xc0, 0x45, 0x00, 0x64, 0x2e, 0x02, 0x70, 0xad, 0xf0, 0x8a, 0x66, 0xfe, - 0x77, 0x12, 0xe9, 0x5b, 0xd4, 0xbe, 0xe3, 0x5b, 0x76, 0x97, 0xdc, 0xa6, 0xdb, 0xce, 0x1e, 0x71, - 0x93, 0x2e, 0x19, 0xcf, 0x9c, 0x67, 0x22, 0x2f, 0x55, 0xe6, 0x59, 0xe5, 0x44, 0xf3, 0xac, 0xfa, - 0x0c, 0xcf, 0x33, 0xf3, 0xe3, 0x32, 0x9c, 0x1b, 0x5f, 0xb3, 0xbc, 0xee, 0xf8, 0x2c, 0xf4, 0xff, - 0xe1, 0xdc, 0xbb, 0x08, 0x91, 0xfb, 0x5e, 0xdc, 0x71, 0xa8, 0x4b, 0x22, 0xbd, 0x0c, 0x6b, 0x96, - 0x99, 0xae, 0x59, 0x52, 0xa0, 0x1b, 0x1b, 0xf7, 0xbd, 0x78, 0x9d, 0x35, 0xe2, 0xeb, 0xd4, 0x79, - 0xe6, 0x09, 0x49, 0xb1, 0xa1, 0x61, 0x5d, 0x6b, 0x57, 0x33, 0xf8, 0x71, 0x46, 0x57, 0x3e, 0x0f, - 0xa3, 0xab, 0x27, 0x62, 0x34, 0x3a, 0x11, 0xa3, 0xa7, 0x4f, 0xc6, 0xe8, 0x99, 0xa7, 0xdc, 0x39, - 0x5c, 0x84, 0x1d, 0xea, 0xc7, 0x96, 0xe7, 0x93, 0xb0, 0x13, 0xc5, 0x56, 0x9c, 0xb0, 0xad, 0xa3, - 0x06, 0xc3, 0xb0, 0x00, 0xc3, 0xb0, 0x9e, 0x8a, 0xb7, 0x41, 0xda, 0x32, 0x06, 0x7d, 0xe3, 0x82, - 0xa3, 0x82, 0xca, 0x0e, 0x31, 0xff, 0x98, 0x10, 0xbf, 0x8c, 0x8a, 0x8e, 0x95, 0x44, 0x44, 0x9f, - 0x5a, 0xd1, 0x56, 0x67, 0xd6, 0x10, 0x37, 0xcc, 0x10, 0x4e, 0x67, 0x10, 0xca, 0x74, 0x06, 0xa0, - 0xee, 0xa2, 0x19, 0x75, 0xd4, 0xe5, 0x2d, 0xa5, 0x7a, 0xbc, 0x2d, 0xa5, 0xf8, 0xc4, 0x2d, 0xe5, - 0x8f, 0x05, 0x84, 0xd9, 0x01, 0x2f, 0x24, 0x4c, 0xf4, 0x35, 0xc8, 0xe8, 0x5f, 0x46, 0xd5, 0x90, - 0xbc, 0x97, 0x90, 0x28, 0xa6, 0xa1, 0x3c, 0xaf, 0x33, 0x50, 0x66, 0x67, 0x06, 0x9a, 0x9f, 0x4d, - 0xc0, 0x05, 0xaa, 0x88, 0xd3, 0x78, 0xfd, 0x3b, 0x6a, 0xfd, 0xbb, 0x8c, 0x4a, 0x61, 0xe2, 0x0f, - 0xd3, 0x54, 0x70, 0x38, 0x4c, 0x7c, 0x35, 0x22, 0x00, 0xe0, 0x4d, 0x34, 0x1f, 0x08, 0xde, 0xdd, - 0x25, 0x1d, 0x11, 0x48, 0xbe, 0xeb, 0x5e, 0x1c, 0xf4, 0x8d, 0xf3, 0x43, 0xe1, 0x56, 0x2e, 0xa4, - 0xb3, 0x39, 0x51, 0xce, 0x94, 0xf0, 0xa0, 0x32, 0xca, 0x54, 0x3b, 0xe7, 0xcb, 0x6c, 0x4e, 0x64, - 0xfe, 0x7d, 0x52, 0x5c, 0x95, 0x3b, 0x0e, 0x21, 0xee, 0x78, 0xa4, 0xc7, 0x47, 0xdc, 0x13, 0x1e, - 0x71, 0xff, 0x5c, 0x85, 0x23, 0xee, 0x9d, 0xd8, 0xeb, 0x7a, 0x11, 0xd4, 0x70, 0xc6, 0x54, 0xfa, - 0x82, 0xa8, 0xf4, 0x73, 0x0d, 0x2d, 0xde, 0xb0, 0xee, 0xb7, 0x45, 0xf9, 0x2b, 0x7a, 0x8d, 0x86, - 0xb7, 0x48, 0xe8, 0x51, 0x57, 0xe4, 0x54, 0x57, 0xd3, 0x9c, 0x2a, 0x3f, 0x18, 0x8d, 0x91, 0x5a, - 0x3c, 0xc9, 0xba, 0x34, 0xe8, 0x1b, 0xc6, 0x48, 0xb9, 0xe4, 0xc7, 0xe8, 0x6e, 0x4f, 0xfb, 0x59, - 0x00, 0xff, 0x4c, 0x43, 0x4b, 0x31, 0x8d, 0xad, 0x6e, 0xc7, 0x49, 0x7a, 0x49, 0xd7, 0x82, 0x65, - 0x37, 0x89, 0xac, 0x5d, 0x96, 0xe5, 0xb0, 0x88, 0xaf, 0x1d, 0x19, 0xf1, 0xdb, 0x4c, 0x6d, 0x3d, - 0xd3, 0xba, 0xc3, 0x94, 0x78, 0xc0, 0xcd, 0x41, 0xdf, 0x58, 0x8e, 0x47, 0x88, 0x25, 0x37, 0x16, - 0x46, 0xc9, 0xeb, 0xbf, 0xd7, 0x50, 0xfd, 0xe8, 0x91, 0x3c, 0x5e, 0xe2, 0xf4, 0x03, 0x39, 0x71, - 0xaa, 0xad, 0x35, 0x1a, 0xbc, 0xd0, 0xda, 0x90, 0x0b, 0xad, 0x8d, 0xe0, 0x60, 0x17, 0x3e, 0x2c, - 0x2d, 0xb4, 0x36, 0xde, 0x4a, 0x2c, 0x3f, 0xf6, 0xe2, 0xc3, 0x27, 0x25, 0x5a, 0xf5, 0xdf, 0x69, - 0xe8, 0xfc, 0x91, 0x9f, 0xfe, 0x2c, 0x78, 0x68, 0x7e, 0x56, 0x40, 0x4b, 0x5b, 0xd4, 0x6e, 0x93, - 0x20, 0xf4, 0x68, 0xe8, 0xc5, 0xde, 0xfb, 0x5f, 0x83, 0x74, 0xf0, 0x3b, 0x68, 0xca, 0x27, 0xf7, - 0x3a, 0xe2, 0x93, 0x0f, 0x61, 0xd1, 0xd2, 0xe0, 0x7c, 0xb5, 0xe8, 0x93, 0x7b, 0xb7, 0x04, 0x2c, - 0x69, 0xd6, 0x24, 0x58, 0x4d, 0x26, 0x4b, 0xc7, 0x4e, 0x26, 0x3f, 0x2d, 0x40, 0x71, 0x51, 0x8a, - 0xf4, 0xe9, 0x4f, 0x33, 0xbe, 0x92, 0x40, 0x7f, 0xc2, 0x4f, 0x37, 0xeb, 0x96, 0xef, 0x90, 0x6e, - 0x77, 0x7c, 0xba, 0x39, 0x22, 0x4e, 0x4f, 0x77, 0x6b, 0x61, 0x7e, 0xcc, 0x1f, 0x93, 0x88, 0xa8, - 0x9e, 0x7e, 0xea, 0x7e, 0x29, 0x41, 0xfd, 0xeb, 0x24, 0x50, 0xf5, 0x36, 0x09, 0x7b, 0x9e, 0x6f, - 0x8d, 0x4f, 0x98, 0xcf, 0x76, 0xb1, 0xf5, 0xcb, 0x39, 0x42, 0x48, 0x14, 0xaa, 0x1c, 0x83, 0x42, - 0x1f, 0x4e, 0xa1, 0x29, 0x60, 0xcd, 0x0d, 0x12, 0xb1, 0xd4, 0x02, 0xdf, 0x44, 0xd5, 0x28, 0x7d, - 0xf1, 0x05, 0xfc, 0xa9, 0xad, 0x2d, 0xa5, 0x79, 0x99, 0xfa, 0x14, 0x8c, 0x07, 0x20, 0x6b, 0x3c, - 0x34, 0xfe, 0xc6, 0x99, 0xf6, 0xd0, 0x06, 0x5e, 0x47, 0x25, 0x60, 0x82, 0x2b, 0x52, 0x90, 0xb3, - 0xa9, 0x35, 0xe9, 0xe5, 0x14, 0x77, 0x92, 0x37, 0x53, 0xec, 0x08, 0x55, 0x66, 0xa4, 0x0b, 0x6f, - 0x8f, 0x80, 0x71, 0x92, 0x11, 0xe9, 0x45, 0x12, 0x37, 0xc2, 0x9b, 0xa9, 0x46, 0x38, 0x86, 0x7f, - 0x84, 0x66, 0xe0, 0x7f, 0x9d, 0x50, 0x3c, 0xce, 0xc9, 0x18, 0x29, 0x1b, 0x53, 0x5e, 0xee, 0xb4, - 0x2e, 0x0c, 0xfa, 0xc6, 0xb9, 0xae, 0x8c, 0x2b, 0xa6, 0xa7, 0x15, 0x11, 0x7e, 0x17, 0x71, 0xa0, - 0x43, 0xf8, 0x53, 0x0f, 0xf1, 0x98, 0xec, 0xbc, 0xd2, 0x81, 0xfc, 0x0c, 0x84, 0x8f, 0x6b, 0x57, - 0x82, 0x15, 0xf3, 0x53, 0xb2, 0x04, 0xbf, 0x8e, 0xca, 0x01, 0x7f, 0x54, 0x01, 0xfc, 0x4d, 0xef, - 0x1b, 0x73, 0x6f, 0x2d, 0x04, 0xc3, 0x38, 0xa2, 0x58, 0x4b, 0xb5, 0x99, 0xa1, 0x90, 0xd7, 0xe2, - 0x81, 0xca, 0x92, 0x21, 0xb9, 0x44, 0xcf, 0x0d, 0x89, 0x86, 0xaa, 0x21, 0x01, 0xe2, 0x1e, 0xc2, - 0x09, 0x14, 0x96, 0x3a, 0x31, 0xed, 0x44, 0xa2, 0xb4, 0x04, 0xbc, 0xab, 0xad, 0x5d, 0xcc, 0xb2, - 0xf9, 0x51, 0xa5, 0x27, 0x5e, 0x36, 0x4b, 0x72, 0x22, 0xa5, 0x97, 0xb9, 0xbc, 0x94, 0xb1, 0x60, - 0x07, 0x2e, 0xb8, 0x61, 0x2e, 0x49, 0x2c, 0x90, 0xae, 0xbd, 0x39, 0x0b, 0x78, 0x33, 0x95, 0x05, - 0x1c, 0xe3, 0x04, 0x17, 0xf7, 0x34, 0x30, 0xb9, 0x14, 0x82, 0xcb, 0x17, 0x38, 0x29, 0xc1, 0x05, - 0x96, 0x27, 0xb8, 0x80, 0x71, 0x07, 0x4d, 0x87, 0x72, 0x56, 0x06, 0x07, 0x23, 0x89, 0x55, 0x8f, - 0xa7, 0x6c, 0x9c, 0x55, 0x8a, 0x92, 0xca, 0x2a, 0x45, 0x84, 0xb7, 0x11, 0x72, 0xb2, 0x6c, 0x04, - 0x6e, 0x84, 0x6b, 0x6b, 0xe7, 0x52, 0xeb, 0xb9, 0x3c, 0xa5, 0xa5, 0x0f, 0xfa, 0xc6, 0xc2, 0xb0, - 0xb9, 0x62, 0x57, 0x32, 0xc3, 0xc2, 0xe0, 0xa4, 0x9b, 0x31, 0xdc, 0x9d, 0x4b, 0x61, 0x50, 0x77, - 0x69, 0xb1, 0xc2, 0xa6, 0x98, 0x1a, 0x86, 0x0c, 0xc6, 0x6f, 0xa3, 0x5a, 0x32, 0x3c, 0xb7, 0xe9, - 0xb3, 0x60, 0x52, 0x3f, 0xea, 0x48, 0xc7, 0xb3, 0x38, 0x49, 0x41, 0x31, 0x2b, 0x5b, 0xc2, 0xef, - 0xa0, 0xa9, 0xb4, 0x56, 0xeb, 0xf9, 0x3b, 0x54, 0x9f, 0x57, 0x2d, 0xe7, 0xcb, 0xb4, 0xdc, 0xb2, - 0x37, 0x44, 0x55, 0xcb, 0x92, 0x00, 0x3b, 0x68, 0x26, 0x54, 0x4e, 0x2e, 0x3a, 0x06, 0xdb, 0x17, - 0x46, 0x0c, 0x5d, 0x16, 0xe0, 0xe7, 0x06, 0x7d, 0x43, 0x57, 0xd5, 0x94, 0x1e, 0x72, 0x26, 0x59, - 0xa0, 0x83, 0xf4, 0x06, 0x58, 0x5f, 0x54, 0x03, 0xad, 0x5e, 0x0d, 0x8b, 0x1d, 0x25, 0xc5, 0xd4, - 0x40, 0x67, 0x30, 0xa3, 0x43, 0x90, 0x5d, 0xbd, 0xeb, 0x4b, 0x2a, 0x1d, 0x72, 0x97, 0xf2, 0x9c, - 0x0e, 0xc3, 0xe6, 0x2a, 0x1d, 0x86, 0x78, 0xab, 0x82, 0x4a, 0xf0, 0x7c, 0x39, 0x32, 0x7f, 0x52, - 0x40, 0xb3, 0xb9, 0x12, 0x06, 0xfe, 0x06, 0x9a, 0x84, 0x2d, 0x8c, 0xe7, 0x13, 0x78, 0xd0, 0x37, - 0x66, 0x7c, 0x75, 0xff, 0x02, 0x39, 0x5e, 0x43, 0x95, 0xb4, 0x94, 0x24, 0x6a, 0x09, 0x90, 0x4b, - 0xa4, 0x98, 0x9c, 0x4b, 0xa4, 0x18, 0x6e, 0xa2, 0x72, 0x8f, 0xef, 0x3d, 0x22, 0x9b, 0x80, 0x65, - 0x47, 0x40, 0xf2, 0x0e, 0x29, 0x20, 0x69, 0x83, 0x9b, 0x3c, 0x46, 0xb9, 0x2c, 0xab, 0xa4, 0x14, - 0x9f, 0xa6, 0x92, 0x62, 0xbe, 0x8f, 0x30, 0x04, 0x70, 0x3b, 0x0e, 0x89, 0xd5, 0x4b, 0x37, 0xc7, - 0x15, 0x54, 0xc8, 0xb2, 0xaa, 0xb9, 0x41, 0xdf, 0x98, 0xf2, 0xe4, 0x94, 0xa1, 0xe0, 0xb9, 0xb8, - 0x35, 0xfc, 0x1a, 0xbe, 0xdd, 0xcd, 0x43, 0x87, 0xf2, 0x16, 0xfb, 0xa4, 0x0f, 0x34, 0x7f, 0x55, - 0x40, 0xd3, 0x5b, 0x90, 0x6a, 0xb5, 0x79, 0x62, 0x78, 0x8c, 0x7e, 0x9f, 0x47, 0xc5, 0x7b, 0x56, - 0xec, 0xec, 0x41, 0xaf, 0x15, 0xfe, 0x69, 0x00, 0xc8, 0x9f, 0x06, 0x00, 0x5e, 0x47, 0xb3, 0x3b, - 0x21, 0xed, 0x75, 0x44, 0x77, 0x2c, 0x1d, 0xe2, 0x81, 0x87, 0x55, 0x89, 0x89, 0x84, 0xa3, 0x4a, - 0x3e, 0x34, 0xad, 0x08, 0x86, 0x19, 0xe0, 0xe4, 0x13, 0x33, 0xc0, 0x57, 0xd1, 0x0c, 0x09, 0x43, - 0x1a, 0x6e, 0xee, 0xdc, 0xf0, 0xa2, 0x88, 0x71, 0xb6, 0x08, 0x3e, 0xc2, 0x44, 0x52, 0x25, 0x92, - 0x72, 0x4e, 0xc7, 0xfc, 0xad, 0x86, 0xa6, 0xde, 0x66, 0xfe, 0xa7, 0x31, 0xc9, 0x3c, 0xd0, 0x9e, - 0xe8, 0xc1, 0xc9, 0x92, 0xdc, 0x2b, 0xa8, 0x0c, 0x71, 0xca, 0xe2, 0xc3, 0x77, 0x96, 0x90, 0xf6, - 0x14, 0x85, 0x12, 0x47, 0x2e, 0xbf, 0x89, 0x8a, 0x40, 0x2b, 0x5c, 0x45, 0xc5, 0x0d, 0xe6, 0xfb, - 0xdc, 0x19, 0x5c, 0x43, 0xe5, 0x8d, 0xbb, 0x9e, 0x13, 0x13, 0x77, 0x4e, 0xc3, 0x65, 0x34, 0x71, - 0xf3, 0xe6, 0x8d, 0xb9, 0x02, 0x5e, 0x40, 0x73, 0xaf, 0x12, 0xcb, 0xed, 0x7a, 0x3e, 0xd9, 0xb8, - 0xcf, 0x37, 0x91, 0xb9, 0x09, 0x3c, 0x85, 0x2a, 0x6d, 0xb2, 0x4f, 0xa0, 0xf1, 0xe4, 0xda, 0xa7, - 0x1a, 0x2a, 0xf2, 0x6c, 0x9e, 0xa0, 0xd9, 0xd7, 0x49, 0xcc, 0xf9, 0x00, 0x48, 0x84, 0x71, 0xb6, - 0x5f, 0x65, 0x14, 0xa9, 0x9f, 0x1b, 0xf2, 0x4c, 0xe1, 0xac, 0x79, 0xe9, 0xc7, 0x9f, 0xfc, 0xfb, - 0xd7, 0x85, 0x8b, 0xa6, 0xde, 0xbc, 0xfb, 0xad, 0xe6, 0x3e, 0xb5, 0xaf, 0x44, 0x24, 0x6e, 0x3e, - 0x80, 0xc0, 0x7c, 0xd0, 0x7c, 0xe0, 0xb9, 0x1f, 0x5c, 0xd3, 0x2e, 0xbf, 0xa8, 0xe1, 0x6b, 0xa8, - 0x08, 0xe1, 0xc5, 0x9c, 0xb0, 0x72, 0xa8, 0x8f, 0xb6, 0x3d, 0xf1, 0xd3, 0x82, 0x06, 0xba, 0xa5, - 0x37, 0xe0, 0xa7, 0x0a, 0x78, 0xe9, 0xb1, 0xe4, 0x7e, 0x83, 0x05, 0xa9, 0xce, 0x57, 0x6b, 0xde, - 0x68, 0x7d, 0x8f, 0x38, 0x07, 0x6d, 0x12, 0x05, 0xd4, 0x8f, 0x48, 0xeb, 0x9d, 0x7f, 0x3c, 0x5c, - 0xd6, 0x3e, 0x7a, 0xb8, 0xac, 0xfd, 0xeb, 0xe1, 0xb2, 0xf6, 0x8b, 0x47, 0xcb, 0x67, 0x3e, 0x7a, - 0xb4, 0x7c, 0xe6, 0x9f, 0x8f, 0x96, 0xcf, 0xfc, 0xf0, 0x9b, 0xbb, 0x5e, 0xbc, 0x97, 0xd8, 0x0d, - 0x87, 0xf6, 0x9a, 0x56, 0xd8, 0xb3, 0x5c, 0x2b, 0x08, 0x29, 0x0b, 0x90, 0xf8, 0x2b, 0xfd, 0x05, - 0xc3, 0x9f, 0x0a, 0x0b, 0xd7, 0x01, 0xb8, 0xc5, 0xc5, 0x8d, 0x4d, 0xda, 0xb8, 0x1e, 0x78, 0x76, - 0x09, 0x7c, 0xb8, 0xfa, 0xbf, 0x00, 0x00, 0x00, 0xff, 0xff, 0xaf, 0x03, 0x5f, 0xa9, 0xa0, 0x31, - 0x00, 0x00, + // 2292 bytes of a gzipped FileDescriptorProto + 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xec, 0x5b, 0xcd, 0x6f, 0x1b, 0xc7, + 0x15, 0xf7, 0x52, 0xfc, 0x1c, 0xea, 0x73, 0x2c, 0xc9, 0x6b, 0x3a, 0xd6, 0x0a, 0x34, 0xd0, 0x2a, + 0x46, 0x4c, 0xa6, 0x74, 0x52, 0x04, 0x46, 0x81, 0xc2, 0x54, 0x94, 0x44, 0x6a, 0x5c, 0x3b, 0x94, + 0x8d, 0xa4, 0x45, 0x00, 0x76, 0x3f, 0x46, 0xd2, 0x4a, 0xe4, 0xce, 0x66, 0x3f, 0x6c, 0x2b, 0x46, + 0x2e, 0xed, 0xa5, 0x87, 0xa2, 0xe9, 0x07, 0x7a, 0x2b, 0xda, 0xa2, 0xc7, 0x02, 0xbd, 0xf4, 0x2f, + 0xe8, 0xa1, 0x28, 0x7a, 0x0c, 0x92, 0x4b, 0x4f, 0x44, 0x61, 0x17, 0x29, 0xc0, 0x53, 0xef, 0xbd, + 0x14, 0xf3, 0x66, 0x76, 0x39, 0xb3, 0xa6, 0x60, 0x59, 0xf9, 0x32, 0x14, 0x9e, 0x6c, 0xfd, 0xde, + 0xbc, 0x37, 0x6f, 0xde, 0xfe, 0x66, 0xe6, 0xcd, 0xbc, 0x21, 0x3a, 0xeb, 0x1f, 0xec, 0x36, 0x4d, + 0xdf, 0x6d, 0x92, 0xbb, 0xc4, 0x8b, 0x1a, 0x7e, 0x40, 0x23, 0x8a, 0xa7, 0x4c, 0xdf, 0xad, 0x19, + 0xbb, 0x94, 0xee, 0xf6, 0x48, 0x13, 0x20, 0x2b, 0xde, 0x69, 0x46, 0x6e, 0x9f, 0x84, 0x91, 0xd9, + 0xf7, 0x79, 0xab, 0xda, 0x62, 0xa2, 0x1a, 0xc6, 0x56, 0xdf, 0x8d, 0xb2, 0xe8, 0x1e, 0x31, 0x7b, + 0xd1, 0x9e, 0x40, 0x2f, 0x64, 0x8d, 0x91, 0xbe, 0x1f, 0x1d, 0x0a, 0xe1, 0x73, 0x42, 0xc8, 0xb4, + 0x4c, 0xcf, 0xa3, 0x91, 0x19, 0xb9, 0xd4, 0x0b, 0x85, 0xf4, 0xa5, 0x83, 0x57, 0xc2, 0x86, 0x4b, + 0x99, 0xb4, 0x6f, 0xda, 0x7b, 0xae, 0x47, 0x82, 0xc3, 0x66, 0xd2, 0x49, 0x40, 0x42, 0x1a, 0x07, + 0x36, 0x69, 0xee, 0x12, 0x8f, 0x04, 0x66, 0x44, 0x1c, 0xae, 0x55, 0xff, 0x6d, 0x0e, 0x2d, 0x6c, + 0x51, 0x6b, 0x1b, 0x5c, 0x8b, 0x88, 0xb3, 0xc1, 0x86, 0x87, 0x2f, 0xa3, 0xe2, 0x3e, 0xb5, 0xba, + 0xae, 0xa3, 0x6b, 0xab, 0xda, 0x5a, 0xa5, 0x7d, 0x76, 0x38, 0x30, 0xe6, 0xf6, 0xa9, 0xb5, 0xe9, + 0xbc, 0x40, 0xfb, 0x6e, 0x04, 0x4e, 0x75, 0x0a, 0x00, 0xe0, 0x97, 0x10, 0x62, 0x6d, 0x43, 0x12, + 0xb1, 0xf6, 0x39, 0x68, 0xbf, 0x3c, 0x1c, 0x18, 0x78, 0x9f, 0x5a, 0xdb, 0x24, 0x52, 0x54, 0xca, + 0x09, 0x86, 0x9f, 0x47, 0x85, 0xf7, 0x62, 0x12, 0x13, 0x7d, 0x6a, 0xd4, 0x01, 0x00, 0x72, 0x07, + 0x00, 0xe0, 0xef, 0xa1, 0x92, 0x1d, 0x10, 0xe6, 0xb3, 0x9e, 0x5f, 0xd5, 0xd6, 0xaa, 0xad, 0x5a, + 0x83, 0x07, 0xa2, 0x91, 0x44, 0xa9, 0x71, 0x3b, 0x09, 0x79, 0x7b, 0x69, 0x38, 0x30, 0x16, 0x44, + 0x73, 0xc9, 0x54, 0x62, 0x01, 0x5f, 0x41, 0x53, 0xfb, 0xd4, 0xd2, 0x0b, 0x60, 0xa8, 0xdc, 0x30, + 0x7d, 0xb7, 0xb1, 0x45, 0xad, 0xf6, 0xc2, 0x70, 0x60, 0xcc, 0xec, 0x53, 0x4b, 0x52, 0x61, 0xed, + 0xea, 0x43, 0x0d, 0xcd, 0x6e, 0x51, 0xeb, 0x2d, 0xe6, 0xc8, 0x69, 0x8f, 0x4d, 0xfd, 0x8f, 0x39, + 0x18, 0xec, 0x9b, 0xc4, 0x0c, 0x4f, 0x3f, 0x11, 0xbe, 0x8d, 0x90, 0xdd, 0x8b, 0xc3, 0x88, 0x04, + 0xcc, 0xdb, 0x02, 0x74, 0x7e, 0x6e, 0x38, 0x30, 0xce, 0x0a, 0x54, 0x71, 0xb7, 0x92, 0x82, 0xf5, + 0x5f, 0xe6, 0xd1, 0x52, 0x12, 0xa4, 0x0e, 0x89, 0xe2, 0xc0, 0x9b, 0xc4, 0xea, 0x88, 0x58, 0xe1, + 0x17, 0x50, 0x31, 0x20, 0x66, 0x48, 0x3d, 0xbd, 0x08, 0x3a, 0x8b, 0xc3, 0x81, 0x31, 0xcf, 0x11, + 0x49, 0x41, 0xb4, 0xc1, 0xdf, 0x45, 0x33, 0x07, 0xb1, 0x45, 0x02, 0x8f, 0x44, 0x24, 0x64, 0x1d, + 0x95, 0x40, 0xa9, 0x36, 0x1c, 0x18, 0xcb, 0x23, 0x81, 0xd2, 0xd7, 0xb4, 0x8c, 0x33, 0x37, 0x7d, + 0xea, 0x74, 0xbd, 0xb8, 0x6f, 0x91, 0x40, 0x2f, 0xaf, 0x6a, 0x6b, 0x05, 0xee, 0xa6, 0x4f, 0x9d, + 0xef, 0x03, 0x28, 0xbb, 0x99, 0x82, 0xac, 0xe3, 0x20, 0xf6, 0xba, 0x66, 0x04, 0x22, 0xe2, 0xe8, + 0x95, 0x55, 0x6d, 0xad, 0xcc, 0x3b, 0x0e, 0x62, 0xef, 0x7a, 0x82, 0xcb, 0x1d, 0xcb, 0x78, 0xfd, + 0xbf, 0x1a, 0x5a, 0x4c, 0x38, 0xb1, 0x71, 0xdf, 0x77, 0x83, 0xd3, 0xbf, 0x56, 0x7c, 0x98, 0x47, + 0x73, 0x5b, 0xd4, 0xba, 0x45, 0x3c, 0xc7, 0xf5, 0x76, 0x27, 0x13, 0x60, 0xfc, 0x04, 0x78, 0x8c, + 0xd2, 0xc5, 0xcf, 0x44, 0xe9, 0xd2, 0xb1, 0x29, 0xfd, 0x22, 0x2a, 0x83, 0x9e, 0xd9, 0x27, 0x30, + 0x11, 0x2a, 0x7c, 0x88, 0xac, 0x81, 0xd9, 0x97, 0xa3, 0x55, 0x12, 0x10, 0x73, 0x35, 0xd1, 0x08, + 0x7d, 0xd3, 0x26, 0x30, 0x09, 0x84, 0xab, 0xa2, 0x0d, 0xe0, 0xb2, 0xab, 0x32, 0x5e, 0xff, 0x1b, + 0x67, 0x44, 0x27, 0xf6, 0xbc, 0x09, 0x23, 0xbe, 0x38, 0x46, 0x5c, 0x45, 0x15, 0x8f, 0x3a, 0x84, + 0x7f, 0xda, 0xd2, 0x28, 0x4a, 0x0c, 0xcc, 0x7c, 0xdb, 0x72, 0x82, 0x9d, 0x78, 0x65, 0x94, 0x69, + 0x54, 0x39, 0x19, 0x8d, 0xd0, 0x53, 0xd2, 0xe8, 0x2f, 0x45, 0x74, 0x76, 0x8b, 0x5a, 0x9b, 0xde, + 0x6e, 0x40, 0xc2, 0x70, 0xd3, 0xdb, 0xa1, 0x13, 0x2a, 0x9d, 0x36, 0x2a, 0xa1, 0x93, 0x51, 0xa9, + 0xfa, 0x74, 0x54, 0xc2, 0x0f, 0xd0, 0x82, 0xcb, 0x69, 0xd4, 0x35, 0x1d, 0x87, 0xfd, 0x4b, 0x42, + 0xbd, 0xb2, 0x3a, 0xb5, 0x56, 0x6d, 0x35, 0x92, 0xcc, 0x3f, 0xcb, 0xb3, 0x86, 0x00, 0xae, 0x27, + 0x0a, 0x1b, 0x5e, 0x14, 0x1c, 0xb6, 0x57, 0x86, 0x03, 0xa3, 0xe6, 0x66, 0x44, 0x52, 0xc7, 0xf3, + 0x59, 0x59, 0xed, 0x00, 0x2d, 0x8d, 0x35, 0x85, 0x2f, 0xa1, 0xa9, 0x03, 0x72, 0x08, 0x2c, 0x2e, + 0xf0, 0x73, 0xc7, 0x01, 0x39, 0x94, 0xcf, 0x1d, 0x07, 0xe4, 0x90, 0x71, 0xf1, 0xae, 0xd9, 0x8b, + 0x89, 0x20, 0x2f, 0x70, 0x11, 0x00, 0x99, 0x8b, 0x00, 0x5c, 0xcb, 0xbd, 0xa2, 0xd5, 0xff, 0x97, + 0x47, 0xfa, 0x16, 0xb5, 0xee, 0x78, 0xa6, 0xd5, 0x23, 0xb7, 0xe9, 0xb6, 0xbd, 0x47, 0x9c, 0xb8, + 0x47, 0x26, 0x33, 0xe7, 0x99, 0xc8, 0x4b, 0x95, 0x79, 0x56, 0x3e, 0xd1, 0x3c, 0xab, 0x3c, 0xc3, + 0xf3, 0xac, 0xfe, 0x71, 0x09, 0xce, 0x8d, 0xaf, 0x99, 0x6e, 0x6f, 0x72, 0x16, 0xfa, 0x7c, 0x38, + 0xf7, 0x2e, 0x42, 0xe4, 0xbe, 0x1b, 0x75, 0x6d, 0xea, 0x90, 0x50, 0x2f, 0xc1, 0x9a, 0x55, 0x4f, + 0xd6, 0x2c, 0x29, 0xd0, 0x8d, 0x8d, 0xfb, 0x6e, 0xb4, 0xce, 0x1a, 0xf1, 0x75, 0xea, 0x3c, 0xf3, + 0x84, 0x24, 0xd8, 0xc8, 0xb0, 0xae, 0x75, 0x2a, 0x29, 0xfc, 0x38, 0xa3, 0xcb, 0x9f, 0x85, 0xd1, + 0x95, 0x13, 0x31, 0x1a, 0x9d, 0x88, 0xd1, 0x33, 0x27, 0x63, 0xf4, 0xec, 0x53, 0xee, 0x1c, 0x0e, + 0xc2, 0x36, 0xf5, 0x22, 0xd3, 0xf5, 0x48, 0xd0, 0x0d, 0x23, 0x33, 0x8a, 0xd9, 0xd6, 0x51, 0x85, + 0xcf, 0xb0, 0x08, 0x9f, 0x61, 0x3d, 0x11, 0x6f, 0x83, 0xb4, 0x6d, 0x0c, 0x07, 0xc6, 0x05, 0x5b, + 0x05, 0x95, 0x1d, 0x62, 0xe1, 0x31, 0x21, 0x7e, 0x19, 0x15, 0x6c, 0x33, 0x0e, 0x89, 0x3e, 0xbd, + 0xaa, 0xad, 0xcd, 0xb6, 0x10, 0x37, 0xcc, 0x10, 0x4e, 0x67, 0x10, 0xca, 0x74, 0x06, 0xa0, 0xe6, + 0xa0, 0x59, 0xf5, 0xab, 0xcb, 0x5b, 0x4a, 0xe5, 0x78, 0x5b, 0x4a, 0xe1, 0x89, 0x5b, 0xca, 0x27, + 0x39, 0x84, 0xd9, 0x01, 0x2f, 0x20, 0x4c, 0xf4, 0x35, 0xc8, 0xe8, 0x5f, 0x46, 0x95, 0x80, 0xbc, + 0x17, 0x93, 0x30, 0xa2, 0x81, 0x3c, 0xaf, 0x53, 0x50, 0x66, 0x67, 0x0a, 0x3e, 0xdd, 0xbc, 0xae, + 0xff, 0x26, 0x0f, 0xd7, 0xad, 0x22, 0xaa, 0x93, 0xd5, 0xf2, 0xa8, 0xd5, 0xf2, 0x32, 0x2a, 0x06, + 0xb1, 0x37, 0x4a, 0x6a, 0xc1, 0xe1, 0x20, 0xf6, 0xd4, 0x88, 0x00, 0x80, 0x37, 0xd1, 0x82, 0x2f, + 0x58, 0x7a, 0x97, 0x74, 0x45, 0x20, 0xf9, 0x1e, 0x7d, 0x71, 0x38, 0x30, 0xce, 0x8f, 0x84, 0x5b, + 0x99, 0x90, 0xce, 0x65, 0x44, 0x19, 0x53, 0xc2, 0x83, 0xf2, 0x38, 0x53, 0x9d, 0x8c, 0x2f, 0x73, + 0x19, 0x91, 0xc4, 0x8b, 0xca, 0x31, 0x78, 0xf1, 0xf7, 0xbc, 0xb8, 0x86, 0xb7, 0x6d, 0x42, 0x9c, + 0x09, 0x2f, 0x26, 0xc7, 0xe7, 0x13, 0x1e, 0x9f, 0xff, 0x5c, 0x81, 0xe3, 0xf3, 0x9d, 0xc8, 0xed, + 0xb9, 0x21, 0xd4, 0x87, 0x26, 0x54, 0xfa, 0x82, 0xa8, 0xf4, 0x73, 0x0d, 0x2d, 0xdd, 0x30, 0xef, + 0x77, 0x44, 0x69, 0x2d, 0x7c, 0x8d, 0x06, 0xb7, 0x48, 0xe0, 0x52, 0x47, 0xe4, 0x6b, 0x57, 0x93, + 0x7c, 0x2d, 0xfb, 0x31, 0x1a, 0x63, 0xb5, 0x78, 0x02, 0x77, 0x69, 0x38, 0x30, 0x8c, 0xb1, 0x72, + 0xc9, 0x8f, 0xf1, 0xdd, 0x9e, 0xf6, 0x73, 0x06, 0xfe, 0x99, 0x86, 0x96, 0x23, 0x1a, 0x99, 0xbd, + 0xae, 0x1d, 0xf7, 0xe3, 0x9e, 0x09, 0x8b, 0x74, 0x1c, 0x9a, 0xbb, 0x2c, 0x83, 0x62, 0x11, 0x6f, + 0x1d, 0x19, 0xf1, 0xdb, 0x4c, 0x6d, 0x3d, 0xd5, 0xba, 0xc3, 0x94, 0x78, 0xc0, 0xeb, 0xc3, 0x81, + 0xb1, 0x12, 0x8d, 0x11, 0x4b, 0x6e, 0x2c, 0x8e, 0x93, 0xd7, 0xfe, 0xa0, 0xa1, 0xda, 0xd1, 0x5f, + 0xf2, 0x78, 0x49, 0xd9, 0x0f, 0xe4, 0xa4, 0xac, 0xda, 0x6a, 0x34, 0x78, 0x11, 0xb7, 0x21, 0x17, + 0x71, 0x1b, 0xfe, 0xc1, 0x2e, 0x0c, 0x2c, 0x29, 0xe2, 0x36, 0xde, 0x8a, 0x4d, 0x2f, 0x72, 0xa3, + 0xc3, 0x27, 0x25, 0x71, 0xb5, 0xdf, 0x6b, 0xe8, 0xfc, 0x91, 0x43, 0x7f, 0x16, 0x3c, 0xac, 0xff, + 0x27, 0x87, 0x96, 0xb7, 0xa8, 0xd5, 0x21, 0x7e, 0xe0, 0xd2, 0xc0, 0x8d, 0xdc, 0xf7, 0xbf, 0x06, + 0xa9, 0xe6, 0x77, 0xd0, 0xb4, 0x47, 0xee, 0x75, 0xc5, 0x90, 0x0f, 0x61, 0xd1, 0xd2, 0xe0, 0xec, + 0xb6, 0xe4, 0x91, 0x7b, 0xb7, 0x04, 0x2c, 0x69, 0x56, 0x25, 0x58, 0x4d, 0x54, 0x8b, 0xc7, 0x4d, + 0x54, 0xeb, 0x9f, 0xe6, 0xa0, 0x70, 0x29, 0x45, 0xfa, 0xf4, 0xa7, 0x19, 0x5f, 0x49, 0xa0, 0xc5, + 0xc9, 0x69, 0xdd, 0xf4, 0x6c, 0xd2, 0xeb, 0x4d, 0x4e, 0x4e, 0x9f, 0xcf, 0xc9, 0xe9, 0x63, 0xfe, + 0x50, 0x45, 0x44, 0xf5, 0xf4, 0x53, 0xf7, 0x4b, 0x09, 0xea, 0x5f, 0xf3, 0x40, 0xd5, 0xdb, 0x24, + 0xe8, 0xbb, 0x9e, 0x39, 0x39, 0x8f, 0x3e, 0xdb, 0x85, 0xdc, 0x2f, 0xe7, 0x08, 0x21, 0x51, 0xa8, + 0x7c, 0x0c, 0x0a, 0x7d, 0x38, 0x8d, 0xa6, 0x81, 0x35, 0x37, 0x48, 0xc8, 0x52, 0x0b, 0x7c, 0x13, + 0x55, 0xc2, 0xe4, 0x35, 0x19, 0xf0, 0xa7, 0xda, 0x5a, 0x4e, 0xf2, 0x32, 0xf5, 0x99, 0x19, 0x0f, + 0x40, 0xda, 0x78, 0x64, 0xfc, 0x8d, 0x33, 0x9d, 0x91, 0x0d, 0xbc, 0x8e, 0x8a, 0xc0, 0x04, 0x47, + 0xa4, 0x20, 0x67, 0x13, 0x6b, 0xd2, 0xab, 0x2c, 0xee, 0x24, 0x6f, 0xa6, 0xd8, 0x11, 0xaa, 0xcc, + 0x48, 0x0f, 0xde, 0x35, 0x01, 0xe3, 0x24, 0x23, 0xd2, 0x6b, 0x27, 0x6e, 0x84, 0x37, 0x53, 0x8d, + 0x70, 0x0c, 0xff, 0x08, 0xcd, 0xc2, 0xff, 0xba, 0x81, 0x78, 0xf8, 0x93, 0x32, 0x52, 0x36, 0xa6, + 0xbc, 0x0a, 0x6a, 0x5f, 0x18, 0x0e, 0x8c, 0x73, 0x3d, 0x19, 0x57, 0x4c, 0xcf, 0x28, 0x22, 0xfc, + 0x2e, 0xe2, 0x40, 0x97, 0xf0, 0x67, 0x24, 0xe2, 0xa1, 0xda, 0x79, 0xa5, 0x03, 0xf9, 0x89, 0x09, + 0xff, 0xae, 0x3d, 0x09, 0x56, 0xcc, 0x4f, 0xcb, 0x12, 0xfc, 0x3a, 0x2a, 0xf9, 0xfc, 0xc1, 0x06, + 0xf0, 0x37, 0xb9, 0xcb, 0xcc, 0xbc, 0xe3, 0x10, 0x0c, 0xe3, 0x88, 0x62, 0x2d, 0xd1, 0x66, 0x86, + 0x02, 0x5e, 0xe7, 0x07, 0x2a, 0x4b, 0x86, 0xe4, 0xf2, 0x3f, 0x37, 0x24, 0x1a, 0xaa, 0x86, 0x04, + 0x88, 0xfb, 0x08, 0xc7, 0x50, 0xb4, 0xea, 0x46, 0xb4, 0x1b, 0x8a, 0xb2, 0x15, 0xf0, 0xae, 0xda, + 0xba, 0x98, 0x66, 0xf3, 0xe3, 0xca, 0x5a, 0xbc, 0x24, 0x17, 0x67, 0x44, 0x4a, 0x2f, 0xf3, 0x59, + 0x29, 0x63, 0xc1, 0x0e, 0x5c, 0x9e, 0xc3, 0x5c, 0x92, 0x58, 0x20, 0x5d, 0xa9, 0x73, 0x16, 0xf0, + 0x66, 0x2a, 0x0b, 0x38, 0xc6, 0x09, 0x2e, 0xee, 0x69, 0x60, 0x72, 0x29, 0x04, 0x97, 0x2f, 0x70, + 0x12, 0x82, 0x0b, 0x2c, 0x4b, 0x70, 0x01, 0xe3, 0x2e, 0x9a, 0x09, 0xe4, 0xac, 0x0c, 0x0e, 0x46, + 0x12, 0xab, 0x1e, 0x4f, 0xd9, 0x38, 0xab, 0x14, 0x25, 0x95, 0x55, 0x8a, 0x08, 0x6f, 0x23, 0x64, + 0xa7, 0xd9, 0x08, 0xdc, 0x36, 0x57, 0x5b, 0xe7, 0x12, 0xeb, 0x99, 0x3c, 0xa5, 0xad, 0x0f, 0x07, + 0xc6, 0xe2, 0xa8, 0xb9, 0x62, 0x57, 0x32, 0xc3, 0xc2, 0x60, 0x27, 0x9b, 0x31, 0xdc, 0xcb, 0x4b, + 0x61, 0x50, 0x77, 0x69, 0xb1, 0xc2, 0x26, 0x98, 0x1a, 0x86, 0x14, 0xc6, 0x6f, 0xa3, 0x6a, 0x3c, + 0x3a, 0xb7, 0xe9, 0x73, 0x60, 0x52, 0x3f, 0xea, 0x48, 0xc7, 0xb3, 0x38, 0x49, 0x41, 0x31, 0x2b, + 0x5b, 0xc2, 0xef, 0xa0, 0xe9, 0xa4, 0x0e, 0xec, 0x7a, 0x3b, 0x54, 0x5f, 0x50, 0x2d, 0x67, 0x4b, + 0xc0, 0xdc, 0xb2, 0x3b, 0x42, 0x55, 0xcb, 0x92, 0x00, 0xdb, 0x68, 0x36, 0x50, 0x4e, 0x2e, 0x3a, + 0x06, 0xdb, 0x17, 0xc6, 0x7c, 0xba, 0x34, 0xc0, 0xcf, 0x0d, 0x07, 0x86, 0xae, 0xaa, 0x29, 0x3d, + 0x64, 0x4c, 0xb2, 0x40, 0xfb, 0xc9, 0x7d, 0xb1, 0xbe, 0xa4, 0x06, 0x5a, 0xbd, 0x48, 0x16, 0x3b, + 0x4a, 0x82, 0xa9, 0x81, 0x4e, 0x61, 0x46, 0x07, 0x3f, 0xbd, 0xd6, 0xd7, 0x97, 0x55, 0x3a, 0x64, + 0x2e, 0xfc, 0x39, 0x1d, 0x46, 0xcd, 0x55, 0x3a, 0x8c, 0xf0, 0x76, 0x19, 0x15, 0xe1, 0x69, 0x74, + 0x58, 0xff, 0x49, 0x0e, 0xcd, 0x65, 0xca, 0x23, 0xf8, 0x1b, 0x28, 0x0f, 0x5b, 0x18, 0xcf, 0x27, + 0xf0, 0x70, 0x60, 0xcc, 0x7a, 0xea, 0xfe, 0x05, 0x72, 0xdc, 0x42, 0xe5, 0xa4, 0x4c, 0x25, 0xea, + 0x14, 0x90, 0x4b, 0x24, 0x98, 0x9c, 0x4b, 0x24, 0x18, 0x6e, 0xa2, 0x52, 0x9f, 0xef, 0x3d, 0x22, + 0x9b, 0x80, 0x65, 0x47, 0x40, 0xf2, 0x0e, 0x29, 0x20, 0x69, 0x83, 0xcb, 0x1f, 0xa3, 0x14, 0x97, + 0x56, 0x69, 0x0a, 0x4f, 0x53, 0xa5, 0xa9, 0xbf, 0x8f, 0x30, 0x04, 0x70, 0x3b, 0x0a, 0x88, 0xd9, + 0x4f, 0x36, 0xc7, 0x55, 0x94, 0x4b, 0xb3, 0xaa, 0xf9, 0xe1, 0xc0, 0x98, 0x76, 0xe5, 0x94, 0x21, + 0xe7, 0x3a, 0xb8, 0x3d, 0x1a, 0x0d, 0xdf, 0xee, 0x16, 0xa0, 0x43, 0x79, 0x8b, 0x7d, 0xd2, 0x00, + 0xeb, 0xbf, 0xca, 0xa1, 0x99, 0x2d, 0x48, 0xb5, 0x3a, 0x3c, 0x31, 0x3c, 0x46, 0xbf, 0xcf, 0xa3, + 0xc2, 0x3d, 0x33, 0xb2, 0xf7, 0xa0, 0xd7, 0x32, 0x1f, 0x1a, 0x00, 0xf2, 0xd0, 0x00, 0xc0, 0xeb, + 0x68, 0x6e, 0x27, 0xa0, 0xfd, 0xae, 0xe8, 0x8e, 0xa5, 0x43, 0x3c, 0xf0, 0xb0, 0x2a, 0x31, 0x91, + 0x70, 0x54, 0xc9, 0x87, 0x66, 0x14, 0xc1, 0x28, 0x03, 0xcc, 0x3f, 0x31, 0x03, 0x7c, 0x15, 0xcd, + 0x92, 0x20, 0xa0, 0xc1, 0xe6, 0xce, 0x0d, 0x37, 0x0c, 0x19, 0x67, 0x0b, 0xe0, 0x23, 0x4c, 0x24, + 0x55, 0x22, 0x29, 0x67, 0x74, 0xea, 0xbf, 0xd3, 0xd0, 0xf4, 0xdb, 0xcc, 0xff, 0x24, 0x26, 0xa9, + 0x07, 0xda, 0x13, 0x3d, 0x38, 0x59, 0x92, 0x7b, 0x05, 0x95, 0x20, 0x4e, 0x69, 0x7c, 0xf8, 0xce, + 0x12, 0xd0, 0xbe, 0xa2, 0x50, 0xe4, 0xc8, 0xe5, 0x37, 0x51, 0x01, 0x68, 0x85, 0x2b, 0xa8, 0xb0, + 0xc1, 0x7c, 0x9f, 0x3f, 0x83, 0xab, 0xa8, 0xb4, 0x71, 0xd7, 0xb5, 0x23, 0xe2, 0xcc, 0x6b, 0xb8, + 0x84, 0xa6, 0x6e, 0xde, 0xbc, 0x31, 0x9f, 0xc3, 0x8b, 0x68, 0xfe, 0x55, 0x62, 0x3a, 0x3d, 0xd7, + 0x23, 0x1b, 0xf7, 0xf9, 0x26, 0x32, 0x3f, 0x85, 0xa7, 0x51, 0xb9, 0x43, 0xf6, 0x09, 0x34, 0xce, + 0xb7, 0x3e, 0xd5, 0x50, 0x81, 0x67, 0xf3, 0x04, 0xcd, 0xbd, 0x4e, 0x22, 0xce, 0x07, 0x40, 0x42, + 0x8c, 0xd3, 0xfd, 0x2a, 0xa5, 0x48, 0xed, 0xdc, 0x88, 0x67, 0x0a, 0x67, 0xeb, 0x97, 0x7e, 0xfc, + 0xc9, 0xbf, 0x7f, 0x9d, 0xbb, 0x58, 0xd7, 0x9b, 0x77, 0xbf, 0xd5, 0xdc, 0xa7, 0xd6, 0x95, 0x90, + 0x44, 0xcd, 0x07, 0x10, 0x98, 0x0f, 0x9a, 0x0f, 0x5c, 0xe7, 0x83, 0x6b, 0xda, 0xe5, 0x17, 0x35, + 0x7c, 0x0d, 0x15, 0x20, 0xbc, 0x98, 0x13, 0x56, 0x0e, 0xf5, 0xd1, 0xb6, 0xa7, 0x7e, 0x9a, 0xd3, + 0x40, 0xb7, 0xf8, 0x06, 0xfc, 0x0c, 0x02, 0x2f, 0x3f, 0x96, 0xdc, 0x6f, 0xb0, 0x20, 0xd5, 0xf8, + 0x6a, 0xcd, 0x1b, 0xad, 0xef, 0x11, 0xfb, 0xa0, 0x43, 0x42, 0x9f, 0x7a, 0x21, 0x69, 0xbf, 0xf3, + 0x8f, 0x87, 0x2b, 0xda, 0x47, 0x0f, 0x57, 0xb4, 0x7f, 0x3d, 0x5c, 0xd1, 0x7e, 0xf1, 0x68, 0xe5, + 0xcc, 0x47, 0x8f, 0x56, 0xce, 0xfc, 0xf3, 0xd1, 0xca, 0x99, 0x1f, 0x7e, 0x73, 0xd7, 0x8d, 0xf6, + 0x62, 0xab, 0x61, 0xd3, 0x7e, 0xd3, 0x0c, 0xfa, 0xa6, 0x63, 0xfa, 0x01, 0x65, 0x01, 0x12, 0x7f, + 0x25, 0xbf, 0x8e, 0xf8, 0x53, 0x6e, 0xf1, 0x3a, 0x00, 0xb7, 0xb8, 0xb8, 0xb1, 0x49, 0x1b, 0xd7, + 0x7d, 0xd7, 0x2a, 0x82, 0x0f, 0x57, 0xff, 0x1f, 0x00, 0x00, 0xff, 0xff, 0x55, 0x79, 0x72, 0x3b, + 0xfc, 0x31, 0x00, 0x00, } // Reference imports to suppress errors if they are not otherwise used. @@ -3802,6 +3818,13 @@ func (m *JobPreemptingEvent) MarshalToSizedBuffer(dAtA []byte) (int, error) { _ = i var l int _ = l + if len(m.Reason) > 0 { + i -= len(m.Reason) + copy(dAtA[i:], m.Reason) + i = encodeVarintEvent(dAtA, i, uint64(len(m.Reason))) + i-- + dAtA[i] = 0x32 + } if len(m.Requestor) > 0 { i -= len(m.Requestor) copy(dAtA[i:], m.Requestor) @@ -3865,6 +3888,13 @@ func (m *JobPreemptedEvent) MarshalToSizedBuffer(dAtA []byte) (int, error) { _ = i var l int _ = l + if len(m.Reason) > 0 { + i -= len(m.Reason) + copy(dAtA[i:], m.Reason) + i = encodeVarintEvent(dAtA, i, uint64(len(m.Reason))) + i-- + dAtA[i] = 0x4a + } if len(m.PreemptiveRunId) > 0 { i -= len(m.PreemptiveRunId) copy(dAtA[i:], m.PreemptiveRunId) @@ -5626,6 +5656,10 @@ func (m *JobPreemptingEvent) Size() (n int) { if l > 0 { n += 1 + l + sovEvent(uint64(l)) } + l = len(m.Reason) + if l > 0 { + n += 1 + l + sovEvent(uint64(l)) + } return n } @@ -5667,6 +5701,10 @@ func (m *JobPreemptedEvent) Size() (n int) { if l > 0 { n += 1 + l + sovEvent(uint64(l)) } + l = len(m.Reason) + if l > 0 { + n += 1 + l + sovEvent(uint64(l)) + } return n } @@ -9718,6 +9756,38 @@ func (m *JobPreemptingEvent) Unmarshal(dAtA []byte) error { } m.Requestor = string(dAtA[iNdEx:postIndex]) iNdEx = postIndex + case 6: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Reason", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowEvent + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= uint64(b&0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthEvent + } + postIndex := iNdEx + intStringLen + if postIndex < 0 { + return ErrInvalidLengthEvent + } + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.Reason = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipEvent(dAtA[iNdEx:]) @@ -10028,6 +10098,38 @@ func (m *JobPreemptedEvent) Unmarshal(dAtA []byte) error { } m.PreemptiveRunId = string(dAtA[iNdEx:postIndex]) iNdEx = postIndex + case 9: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Reason", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowEvent + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= uint64(b&0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthEvent + } + postIndex := iNdEx + intStringLen + if postIndex < 0 { + return ErrInvalidLengthEvent + } + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.Reason = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipEvent(dAtA[iNdEx:]) diff --git a/pkg/api/event.proto b/pkg/api/event.proto index 05d1e5fd7ad..553bc93832c 100644 --- a/pkg/api/event.proto +++ b/pkg/api/event.proto @@ -129,6 +129,7 @@ message JobPreemptingEvent { string queue = 3; google.protobuf.Timestamp created = 4; string requestor = 5; + string reason = 6; } message JobPreemptedEvent { @@ -140,6 +141,7 @@ message JobPreemptedEvent { string run_id = 6; string preemptive_job_id = 7; string preemptive_run_id = 8; + string reason = 9; } message JobSucceededEvent { diff --git a/pkg/api/submit.pb.go b/pkg/api/submit.pb.go index 5edf4bd0777..8ffdced2437 100644 --- a/pkg/api/submit.pb.go +++ b/pkg/api/submit.pb.go @@ -464,6 +464,7 @@ type JobPreemptRequest struct { Queue string `protobuf:"bytes,1,opt,name=queue,proto3" json:"queue,omitempty"` JobSetId string `protobuf:"bytes,2,opt,name=job_set_id,json=jobSetId,proto3" json:"jobSetId,omitempty"` JobIds []string `protobuf:"bytes,3,rep,name=job_ids,json=jobIds,proto3" json:"jobIds,omitempty"` + Reason string `protobuf:"bytes,4,opt,name=reason,proto3" json:"reason,omitempty"` } func (m *JobPreemptRequest) Reset() { *m = JobPreemptRequest{} } @@ -520,6 +521,13 @@ func (m *JobPreemptRequest) GetJobIds() []string { return nil } +func (m *JobPreemptRequest) GetReason() string { + if m != nil { + return m.Reason + } + return "" +} + // swagger:model type JobCancelRequest struct { JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId,proto3" json:"jobId,omitempty"` @@ -2227,199 +2235,199 @@ func init() { func init() { proto.RegisterFile("pkg/api/submit.proto", fileDescriptor_e998bacb27df16c1) } var fileDescriptor_e998bacb27df16c1 = []byte{ - // 3060 bytes of a gzipped FileDescriptorProto - 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xec, 0x3a, 0x4b, 0x6c, 0x1b, 0xd7, - 0xb5, 0x1a, 0x51, 0x3f, 0x1e, 0xea, 0x43, 0x5d, 0x7d, 0x3c, 0xa2, 0x1d, 0x51, 0x9e, 0x24, 0x8e, - 0xac, 0xf8, 0x51, 0xb1, 0xf2, 0x82, 0x67, 0xfb, 0xe5, 0x3d, 0x57, 0xa4, 0x68, 0x5b, 0xb2, 0x2d, - 0x2b, 0x92, 0x95, 0x4f, 0x51, 0x94, 0x1d, 0x72, 0xae, 0xa8, 0x91, 0xc8, 0x99, 0xc9, 0xcc, 0x50, - 0xae, 0x5a, 0x64, 0x53, 0x14, 0xc8, 0xa2, 0x9b, 0xa0, 0x5d, 0x16, 0x68, 0x0a, 0xb4, 0xab, 0x74, - 0xdd, 0x4d, 0xd1, 0x45, 0x97, 0x5d, 0x06, 0xe8, 0xa6, 0xdd, 0x10, 0x45, 0xd2, 0x36, 0x00, 0x77, - 0xdd, 0x77, 0x51, 0xdc, 0x73, 0xe7, 0x73, 0x87, 0x1f, 0x91, 0x94, 0xad, 0x64, 0xd3, 0x1d, 0xef, - 0xb9, 0xe7, 0x7f, 0xcf, 0x3d, 0x9f, 0x3b, 0x84, 0x59, 0xeb, 0xb8, 0xbc, 0xaa, 0x5a, 0xfa, 0xaa, - 0x53, 0x2b, 0x56, 0x75, 0x37, 0x63, 0xd9, 0xa6, 0x6b, 0x92, 0x98, 0x6a, 0xe9, 0xa9, 0xcb, 0x65, - 0xd3, 0x2c, 0x57, 0xe8, 0x2a, 0x82, 0x8a, 0xb5, 0x83, 0x55, 0x5a, 0xb5, 0xdc, 0x53, 0x8e, 0x91, - 0x4a, 0x37, 0x6f, 0xba, 0x7a, 0x95, 0x3a, 0xae, 0x5a, 0xb5, 0x3c, 0x04, 0xe5, 0xf8, 0x96, 0x93, - 0xd1, 0x4d, 0xe4, 0x5d, 0x32, 0x6d, 0xba, 0x7a, 0x72, 0x73, 0xb5, 0x4c, 0x0d, 0x6a, 0xab, 0x2e, - 0xd5, 0x3c, 0x9c, 0x65, 0x01, 0xc7, 0xa0, 0xee, 0x33, 0xd3, 0x3e, 0xd6, 0x8d, 0x72, 0x3b, 0xcc, - 0x2b, 0x9e, 0x38, 0x86, 0xa9, 0x1a, 0x86, 0xe9, 0xaa, 0xae, 0x6e, 0x1a, 0x8e, 0xb7, 0x1b, 0x18, - 0x71, 0x48, 0xd5, 0x8a, 0x7b, 0xc8, 0xa1, 0xca, 0xa7, 0x71, 0x98, 0xdd, 0x32, 0x8b, 0x7b, 0x68, - 0xd8, 0x2e, 0xfd, 0xb0, 0x46, 0x1d, 0x77, 0xd3, 0xa5, 0x55, 0xb2, 0x06, 0x63, 0x96, 0xad, 0x9b, - 0xb6, 0xee, 0x9e, 0xca, 0xd2, 0x92, 0xb4, 0x2c, 0x65, 0xe7, 0x1b, 0xf5, 0x34, 0xf1, 0x61, 0x37, - 0xcc, 0xaa, 0xee, 0xa2, 0xad, 0xbb, 0x01, 0x1e, 0x79, 0x0b, 0xe2, 0x86, 0x5a, 0xa5, 0x8e, 0xa5, - 0x96, 0xa8, 0x1c, 0x5b, 0x92, 0x96, 0xe3, 0xd9, 0x4b, 0x8d, 0x7a, 0x7a, 0x26, 0x00, 0x0a, 0x54, - 0x21, 0x26, 0x79, 0x13, 0xe2, 0xa5, 0x8a, 0x4e, 0x0d, 0xb7, 0xa0, 0x6b, 0xf2, 0x18, 0x92, 0xa1, - 0x2c, 0x0e, 0xdc, 0xd4, 0x44, 0x59, 0x3e, 0x8c, 0xec, 0xc1, 0x48, 0x45, 0x2d, 0xd2, 0x8a, 0x23, - 0x0f, 0x2d, 0xc5, 0x96, 0x13, 0x6b, 0xaf, 0x66, 0x54, 0x4b, 0xcf, 0xb4, 0x33, 0x25, 0xf3, 0x08, - 0xf1, 0xf2, 0x86, 0x6b, 0x9f, 0x66, 0x67, 0x1b, 0xf5, 0x74, 0x92, 0x13, 0x0a, 0x6c, 0x3d, 0x56, - 0xa4, 0x0c, 0x09, 0xc1, 0x71, 0xf2, 0x30, 0x72, 0x5e, 0xe9, 0xcc, 0x79, 0x3d, 0x44, 0xe6, 0xec, - 0x17, 0x1a, 0xf5, 0xf4, 0x9c, 0xc0, 0x42, 0x90, 0x21, 0x72, 0x26, 0x1f, 0x4b, 0x30, 0x6b, 0xd3, - 0x0f, 0x6b, 0xba, 0x4d, 0xb5, 0x82, 0x61, 0x6a, 0xb4, 0xe0, 0x19, 0x33, 0x82, 0x22, 0x6f, 0x76, - 0x16, 0xb9, 0xeb, 0x51, 0x6d, 0x9b, 0x1a, 0x15, 0x0d, 0x53, 0x1a, 0xf5, 0xf4, 0x15, 0xbb, 0x65, - 0x33, 0x54, 0x40, 0x96, 0x76, 0x49, 0xeb, 0x3e, 0x79, 0x02, 0x63, 0x96, 0xa9, 0x15, 0x1c, 0x8b, - 0x96, 0xe4, 0xc1, 0x25, 0x69, 0x39, 0xb1, 0x76, 0x39, 0xc3, 0x23, 0x0e, 0x75, 0x60, 0x51, 0x99, - 0x39, 0xb9, 0x99, 0xd9, 0x31, 0xb5, 0x3d, 0x8b, 0x96, 0xf0, 0x3c, 0xa7, 0x2d, 0xbe, 0x88, 0xf0, - 0x1e, 0xf5, 0x80, 0x64, 0x07, 0xe2, 0x3e, 0x43, 0x47, 0x1e, 0x45, 0x73, 0xce, 0xe4, 0xc8, 0xc3, - 0x8a, 0x2f, 0x9c, 0x48, 0x58, 0x79, 0x30, 0x92, 0x83, 0x51, 0xdd, 0x28, 0xdb, 0xd4, 0x71, 0xe4, - 0x38, 0xf2, 0x23, 0xc8, 0x68, 0x93, 0xc3, 0x72, 0xa6, 0x71, 0xa0, 0x97, 0xb3, 0x73, 0x4c, 0x31, - 0x0f, 0x4d, 0xe0, 0xe2, 0x53, 0x92, 0x7b, 0x30, 0xe6, 0x50, 0xfb, 0x44, 0x2f, 0x51, 0x47, 0x06, - 0x81, 0xcb, 0x1e, 0x07, 0x7a, 0x5c, 0x50, 0x19, 0x1f, 0x4f, 0x54, 0xc6, 0x87, 0xb1, 0x18, 0x77, - 0x4a, 0x87, 0x54, 0xab, 0x55, 0xa8, 0x2d, 0x27, 0xc2, 0x18, 0x0f, 0x80, 0x62, 0x8c, 0x07, 0xc0, - 0x94, 0x0a, 0x09, 0xe1, 0xb4, 0xc8, 0xcb, 0x10, 0x3b, 0xa6, 0xfc, 0x62, 0xc5, 0xb3, 0xd3, 0x8d, - 0x7a, 0x7a, 0xe2, 0x98, 0x8a, 0x77, 0x8a, 0xed, 0x92, 0xeb, 0x30, 0x7c, 0xa2, 0x56, 0x6a, 0x14, - 0xcf, 0x25, 0x9e, 0x9d, 0x69, 0xd4, 0xd3, 0x53, 0x08, 0x10, 0x10, 0x39, 0xc6, 0x9d, 0xc1, 0x5b, - 0x52, 0xea, 0x00, 0x92, 0xcd, 0xf1, 0x78, 0x21, 0x72, 0xaa, 0x70, 0xa9, 0x43, 0x10, 0x5e, 0x84, - 0x38, 0xe5, 0x9f, 0x31, 0x98, 0x88, 0x1c, 0x35, 0xb9, 0x03, 0x43, 0xee, 0xa9, 0x45, 0x51, 0xcc, - 0xe4, 0x5a, 0x52, 0x0c, 0x86, 0xa7, 0xa7, 0x16, 0xc5, 0x3b, 0x3e, 0xc9, 0x30, 0x22, 0x01, 0x8a, - 0x34, 0x4c, 0xb8, 0x65, 0xda, 0xae, 0x23, 0x0f, 0x2e, 0xc5, 0x96, 0x27, 0xb8, 0x70, 0x04, 0x88, - 0xc2, 0x11, 0x40, 0xbe, 0x17, 0x4d, 0x06, 0x31, 0x0c, 0x9a, 0x97, 0x5b, 0x43, 0xef, 0xfc, 0x59, - 0xe0, 0x36, 0x24, 0xdc, 0x8a, 0x53, 0xa0, 0x86, 0x5a, 0xac, 0x50, 0x4d, 0x1e, 0x5a, 0x92, 0x96, - 0xc7, 0xb2, 0x72, 0xa3, 0x9e, 0x9e, 0x75, 0x99, 0x47, 0x11, 0x2a, 0xd0, 0x42, 0x08, 0xc5, 0x9c, - 0x49, 0x6d, 0xb7, 0xc0, 0xb2, 0xa8, 0x3c, 0x2c, 0xe4, 0x4c, 0x6a, 0xbb, 0xdb, 0x6a, 0x95, 0x46, - 0x72, 0xa6, 0x07, 0x23, 0x77, 0x61, 0xa2, 0xe6, 0xd0, 0x42, 0xa9, 0x52, 0x73, 0x5c, 0x6a, 0x6f, - 0xee, 0xc8, 0x23, 0x28, 0x31, 0xd5, 0xa8, 0xa7, 0xe7, 0x6b, 0x0e, 0xcd, 0xf9, 0x70, 0x81, 0x78, - 0x5c, 0x84, 0x7f, 0x5d, 0x21, 0xa6, 0xb8, 0x30, 0x11, 0xb9, 0x97, 0xe4, 0x56, 0x9b, 0x23, 0xf7, - 0x30, 0xf0, 0xc8, 0x49, 0xeb, 0x91, 0xf7, 0x7d, 0xe0, 0xca, 0x5f, 0x24, 0x48, 0x36, 0xe7, 0x5c, - 0x46, 0xff, 0x61, 0x8d, 0xd6, 0xa8, 0x67, 0x20, 0xd2, 0x23, 0x40, 0xa4, 0x47, 0x00, 0xf9, 0x6f, - 0x80, 0x23, 0xb3, 0x58, 0x70, 0x28, 0x16, 0xb2, 0xc1, 0xf0, 0x50, 0x8e, 0xcc, 0xe2, 0x1e, 0x6d, - 0x2a, 0x64, 0x3e, 0x8c, 0x68, 0x30, 0xcd, 0xa8, 0x6c, 0x2e, 0xaf, 0xc0, 0x10, 0xfc, 0x60, 0x5b, - 0xe8, 0x58, 0x06, 0xb2, 0x2f, 0x35, 0xea, 0xe9, 0x85, 0x23, 0xb3, 0x28, 0xc0, 0x44, 0x8b, 0xa6, - 0x9a, 0xb6, 0x94, 0x5f, 0x49, 0x30, 0xbd, 0x65, 0x16, 0x77, 0x6c, 0xca, 0x10, 0xbe, 0x36, 0xe3, - 0xfe, 0x0b, 0x46, 0x19, 0x95, 0xae, 0x71, 0x93, 0xe2, 0xbc, 0xfe, 0x1e, 0x99, 0xc5, 0x4d, 0x2d, - 0x52, 0x7f, 0x39, 0x44, 0xf9, 0x17, 0x3f, 0x81, 0x9c, 0x6a, 0x94, 0x68, 0xc5, 0x57, 0x72, 0x05, - 0x46, 0x38, 0x0f, 0x51, 0x4b, 0x24, 0x10, 0xb5, 0x44, 0xc0, 0x39, 0xb5, 0x0c, 0xdc, 0x10, 0xeb, - 0xea, 0x06, 0xc1, 0xa0, 0xa1, 0xee, 0x06, 0x91, 0x1b, 0x30, 0x62, 0x53, 0xd5, 0x31, 0x0d, 0xef, - 0x8e, 0x22, 0x36, 0x87, 0x88, 0xd8, 0x1c, 0xa2, 0xfc, 0x5d, 0x82, 0x99, 0x2d, 0x54, 0x2a, 0xea, - 0x81, 0xa8, 0x55, 0x52, 0xbf, 0x56, 0x0d, 0x76, 0xb5, 0xea, 0x2e, 0x8c, 0x1c, 0xe8, 0x15, 0x97, - 0xda, 0xe8, 0x81, 0xc4, 0xda, 0x74, 0x10, 0x78, 0xd4, 0xbd, 0x87, 0x1b, 0x5c, 0x73, 0x8e, 0x24, - 0x6a, 0xce, 0x21, 0x82, 0x9d, 0x43, 0x3d, 0xd8, 0xf9, 0x10, 0xc6, 0x45, 0xde, 0xe4, 0x7f, 0x61, - 0xc4, 0x71, 0x55, 0x97, 0x3a, 0xb2, 0xb4, 0x14, 0x5b, 0x9e, 0x5c, 0x9b, 0x08, 0xc4, 0x33, 0x28, - 0x67, 0xc6, 0x11, 0x44, 0x66, 0x1c, 0xa2, 0x7c, 0x35, 0x05, 0xb1, 0x2d, 0xb3, 0x48, 0x96, 0x60, - 0x30, 0x70, 0x4e, 0xb2, 0x51, 0x4f, 0x8f, 0xeb, 0xa2, 0x5b, 0x06, 0x75, 0x2d, 0xda, 0x67, 0x4e, - 0xf4, 0xd8, 0x67, 0x5e, 0x78, 0x44, 0x45, 0x9a, 0xe6, 0xd1, 0x9e, 0x9b, 0xe6, 0x6c, 0xd0, 0xff, - 0xf2, 0x9e, 0x68, 0xd6, 0xf7, 0x59, 0x1f, 0xed, 0xee, 0xbb, 0xd1, 0x0a, 0x07, 0xd1, 0xa4, 0x73, - 0xfe, 0xba, 0x76, 0xd2, 0xa1, 0xb9, 0x4d, 0xa0, 0x80, 0xa5, 0x40, 0xc0, 0x8b, 0xee, 0x65, 0xaf, - 0xc3, 0xb0, 0xf9, 0xcc, 0xa0, 0xb6, 0x37, 0x44, 0xa0, 0xd7, 0x11, 0x20, 0x7a, 0x1d, 0x01, 0x84, - 0xc2, 0x65, 0x74, 0x7f, 0x01, 0x97, 0xce, 0xa1, 0x6e, 0x15, 0x6a, 0x0e, 0xb5, 0x0b, 0x65, 0xdb, - 0xac, 0x59, 0x8e, 0x3c, 0x85, 0x77, 0xfb, 0x5a, 0xa3, 0x9e, 0x56, 0x10, 0xed, 0x89, 0x8f, 0xb5, - 0xef, 0x50, 0xfb, 0x3e, 0xe2, 0x08, 0x3c, 0xe5, 0x4e, 0x38, 0xe4, 0xc7, 0x12, 0x5c, 0x2b, 0x99, - 0x55, 0x8b, 0x75, 0x0b, 0x54, 0x2b, 0x9c, 0x25, 0x72, 0x66, 0x49, 0x5a, 0x1e, 0xcf, 0xbe, 0xd1, - 0xa8, 0xa7, 0x6f, 0x84, 0x14, 0xef, 0x74, 0x17, 0xae, 0x74, 0xc7, 0x8e, 0x0c, 0x73, 0x43, 0x3d, - 0x0e, 0x73, 0xe2, 0x60, 0x30, 0xfc, 0xc2, 0x07, 0x83, 0xf1, 0x17, 0x31, 0x18, 0xfc, 0x52, 0x82, - 0x25, 0xaf, 0xc5, 0xd6, 0x8d, 0x72, 0xc1, 0xa6, 0x8e, 0x59, 0xb3, 0x4b, 0xb4, 0xe0, 0x85, 0x46, - 0x95, 0x1a, 0xae, 0x23, 0xcf, 0xa1, 0xee, 0xcb, 0xed, 0x24, 0xed, 0x7a, 0x04, 0xbb, 0x02, 0x7e, - 0xf6, 0x46, 0xa3, 0x9e, 0x5e, 0x0e, 0xb9, 0xb6, 0xc3, 0x11, 0x94, 0x59, 0x3c, 0x1b, 0x93, 0x3c, - 0x84, 0xd1, 0x92, 0x4d, 0xd9, 0x90, 0x8e, 0xcd, 0x56, 0x62, 0x2d, 0x95, 0xe1, 0x53, 0x7a, 0xc6, - 0x7f, 0x14, 0xc8, 0x3c, 0xf5, 0x1f, 0x05, 0xf8, 0x0c, 0xe3, 0xa1, 0x8b, 0x33, 0x8c, 0x07, 0x12, - 0x07, 0xa1, 0xc9, 0x17, 0x32, 0x08, 0x25, 0x9f, 0x63, 0x10, 0xfa, 0x0e, 0x24, 0x8e, 0x6f, 0x39, - 0x05, 0x5f, 0xa1, 0x69, 0x64, 0x75, 0x55, 0x74, 0x73, 0xf8, 0x5a, 0xc1, 0x9c, 0xed, 0x69, 0xc9, - 0xfb, 0xdb, 0xe3, 0x5b, 0xce, 0x66, 0x8b, 0x8a, 0x10, 0x42, 0x59, 0x6a, 0x62, 0xdc, 0x3d, 0x69, - 0x32, 0xe9, 0x1c, 0x2e, 0x9e, 0xde, 0x01, 0x5f, 0x6f, 0xdd, 0xc4, 0xd7, 0x83, 0x46, 0xc7, 0xb7, - 0xd9, 0x5e, 0xc7, 0x37, 0xb2, 0x09, 0xd3, 0xfc, 0xee, 0xba, 0x6e, 0xa5, 0xe0, 0xd0, 0x92, 0x69, - 0x68, 0x8e, 0x3c, 0xbf, 0x24, 0x2d, 0xc7, 0x78, 0x27, 0x86, 0x9b, 0x4f, 0xdd, 0xca, 0x1e, 0xdf, - 0x12, 0x3b, 0xb1, 0xa6, 0xad, 0xff, 0x4c, 0x82, 0xe7, 0x9e, 0x0a, 0xfe, 0x21, 0xc1, 0xfc, 0x16, - 0xeb, 0x6b, 0xbd, 0x1c, 0xa5, 0xff, 0x80, 0xfa, 0x1d, 0x92, 0xd0, 0x96, 0x49, 0x3d, 0xb4, 0x65, - 0x17, 0x5e, 0xd4, 0xdf, 0x86, 0x71, 0x83, 0x3e, 0x2b, 0x34, 0x25, 0x5d, 0xac, 0x9f, 0x06, 0x7d, - 0xb6, 0xd3, 0x9a, 0x77, 0x13, 0x02, 0x58, 0xf9, 0xcd, 0x20, 0x5c, 0x6a, 0x31, 0xd4, 0xb1, 0x4c, - 0xc3, 0xa1, 0xe4, 0xe7, 0x12, 0xc8, 0x76, 0xb8, 0x81, 0x47, 0xcc, 0x32, 0x5f, 0xad, 0xe2, 0x72, - 0xdb, 0x13, 0x6b, 0xb7, 0xfd, 0x02, 0xdb, 0x8e, 0x41, 0x66, 0xb7, 0x89, 0x78, 0x97, 0xd3, 0xf2, - 0xca, 0xfb, 0x6a, 0xa3, 0x9e, 0xbe, 0x6a, 0xb7, 0xc7, 0x10, 0xb4, 0xbd, 0xd4, 0x01, 0x25, 0x65, - 0xc3, 0x95, 0xb3, 0xf8, 0x5f, 0x48, 0x58, 0x18, 0x30, 0x27, 0x8c, 0x48, 0xdc, 0x4a, 0x7c, 0xc2, - 0xec, 0x67, 0x70, 0xb8, 0x0e, 0xc3, 0xd4, 0xb6, 0x4d, 0x5b, 0x94, 0x89, 0x00, 0x11, 0x15, 0x01, - 0xca, 0x47, 0x38, 0x49, 0x45, 0xe5, 0x91, 0x43, 0x20, 0x7c, 0x8a, 0xe3, 0x6b, 0x6f, 0x8c, 0xe3, - 0xe7, 0x91, 0x6a, 0x1e, 0xe3, 0x42, 0x1d, 0xb3, 0x8b, 0x8d, 0x7a, 0x3a, 0x85, 0xc3, 0x5a, 0x08, - 0x14, 0x3d, 0x9d, 0x6c, 0xde, 0x53, 0x3e, 0x4e, 0xc0, 0x30, 0x16, 0x7a, 0x72, 0x0d, 0x86, 0x70, - 0xfc, 0xe7, 0xd6, 0xe1, 0x08, 0x6c, 0x44, 0x47, 0x7f, 0xdc, 0x27, 0x79, 0x98, 0xf2, 0x03, 0xb1, - 0x70, 0xa0, 0x96, 0x5c, 0xcf, 0x4a, 0x29, 0x7b, 0xa5, 0x51, 0x4f, 0xcb, 0xfe, 0xd6, 0x3d, 0xdc, - 0x11, 0x88, 0x27, 0xa3, 0x3b, 0xe4, 0x36, 0x24, 0xb0, 0x5f, 0xe1, 0xed, 0x8b, 0x37, 0xcf, 0x61, - 0xd6, 0x65, 0x60, 0xde, 0x76, 0x88, 0x59, 0x37, 0x84, 0xb2, 0xeb, 0x80, 0x5d, 0x8e, 0x4f, 0xcb, - 0x47, 0x27, 0xbc, 0x0e, 0x08, 0x6f, 0x21, 0x4e, 0x08, 0x60, 0x52, 0x86, 0xa9, 0xa0, 0xb4, 0x57, - 0xf4, 0xaa, 0xee, 0xfa, 0x2f, 0xb3, 0x8b, 0xe8, 0x58, 0x74, 0x46, 0x50, 0xcb, 0x1f, 0x21, 0x02, - 0x8f, 0x66, 0xe6, 0x5c, 0xd9, 0x8e, 0x6c, 0x44, 0x5a, 0x93, 0xc9, 0xe8, 0x1e, 0xf9, 0xad, 0x04, - 0xd7, 0x9a, 0x24, 0x15, 0x8a, 0xa7, 0xc1, 0x2d, 0x2e, 0x94, 0x2a, 0xaa, 0xe3, 0xf0, 0x27, 0x97, - 0x51, 0xe1, 0x9d, 0xb6, 0x9d, 0x02, 0xd9, 0x53, 0xff, 0x36, 0xe7, 0x18, 0xd1, 0xb6, 0x5a, 0xa5, - 0x5c, 0xa7, 0xd5, 0x46, 0x3d, 0xfd, 0xba, 0xdd, 0x0d, 0x57, 0x70, 0xc5, 0xd5, 0xae, 0xc8, 0x64, - 0x0f, 0x12, 0x16, 0xb5, 0xab, 0xba, 0xe3, 0x60, 0x1f, 0xcf, 0xdf, 0x90, 0xe7, 0x05, 0xdd, 0x76, - 0xc2, 0x5d, 0xee, 0x75, 0x01, 0x5d, 0xf4, 0xba, 0x00, 0x66, 0x3d, 0x63, 0xc9, 0xb4, 0x35, 0xd3, - 0xa0, 0xfc, 0x51, 0x7e, 0xcc, 0x1b, 0x96, 0x3c, 0x58, 0x64, 0x58, 0xf2, 0x60, 0xe4, 0x31, 0x4c, - 0xf3, 0x56, 0xbf, 0xa0, 0x51, 0xcb, 0xa6, 0x25, 0xec, 0x7b, 0xe2, 0x78, 0xd8, 0x4b, 0x2c, 0xd0, - 0xf9, 0xe6, 0x46, 0xb0, 0x17, 0x39, 0x8d, 0x64, 0xf3, 0x2e, 0xd9, 0x08, 0x66, 0x1c, 0x68, 0x31, - 0xa9, 0xe7, 0x29, 0x27, 0xf5, 0x95, 0x04, 0x09, 0xc1, 0x01, 0x64, 0x17, 0xc6, 0x9c, 0x5a, 0xf1, - 0x88, 0x96, 0x82, 0x84, 0xb9, 0xd8, 0xde, 0x55, 0x99, 0x3d, 0x8e, 0xe6, 0x35, 0x43, 0x1e, 0x4d, - 0xa4, 0x19, 0xf2, 0x60, 0x98, 0xb2, 0xa8, 0x5d, 0xe4, 0xaf, 0x4c, 0x7e, 0xca, 0x62, 0x80, 0x48, - 0xca, 0x62, 0x80, 0xd4, 0x07, 0x30, 0xea, 0xf1, 0x65, 0x17, 0xf8, 0x58, 0x37, 0x34, 0xf1, 0x02, - 0xb3, 0xb5, 0x78, 0x81, 0xd9, 0x3a, 0xb8, 0xe8, 0x83, 0x67, 0x5f, 0xf4, 0x94, 0x0e, 0x33, 0x6d, - 0xae, 0xc1, 0x39, 0x92, 0xae, 0xd4, 0xb5, 0xf4, 0xff, 0x42, 0x82, 0x6b, 0xbd, 0x45, 0x7c, 0x6f, - 0xe2, 0x1f, 0x8a, 0xe2, 0xfd, 0x19, 0x31, 0xc2, 0xb0, 0x49, 0x5a, 0x37, 0x05, 0x2f, 0xbe, 0xcd, - 0x52, 0x7e, 0x3a, 0x0c, 0x97, 0xcf, 0x50, 0x91, 0x8d, 0x27, 0x0b, 0x55, 0xf5, 0xfb, 0x7a, 0xb5, - 0x56, 0x0d, 0x67, 0x93, 0x03, 0x5b, 0x2d, 0xb1, 0xb2, 0xe8, 0x85, 0xde, 0xff, 0x75, 0x33, 0x34, - 0xf3, 0x98, 0x73, 0xf0, 0xa1, 0xf7, 0x3c, 0x7a, 0xa1, 0x5e, 0x57, 0xdb, 0x63, 0x88, 0xf5, 0xba, - 0x03, 0x0a, 0xf9, 0x9d, 0x04, 0x57, 0x3b, 0xaa, 0x88, 0xb9, 0xcf, 0x34, 0x2b, 0x18, 0xd4, 0x89, - 0xb5, 0xdc, 0x79, 0x55, 0xcd, 0x9e, 0xee, 0x98, 0x66, 0x85, 0x2b, 0xfc, 0x7a, 0xa3, 0x9e, 0x7e, - 0xad, 0x7a, 0x16, 0x9e, 0xa0, 0xf6, 0x4b, 0x67, 0x22, 0xb2, 0x66, 0xe3, 0x2c, 0xe7, 0x5c, 0x54, - 0xdc, 0x2b, 0xdd, 0xcd, 0xec, 0x4d, 0xf4, 0x93, 0x68, 0xcc, 0xbf, 0xd2, 0xea, 0x5f, 0xc6, 0xb0, - 0xbf, 0xb8, 0x57, 0x7e, 0x3f, 0x08, 0xe9, 0x2e, 0x3c, 0xc8, 0xaf, 0x7b, 0x08, 0xcc, 0xf5, 0x5e, - 0xb4, 0xb9, 0xd0, 0xe0, 0xfc, 0x26, 0xce, 0x57, 0xc9, 0x43, 0x1c, 0xeb, 0xc0, 0x23, 0xdd, 0x71, - 0xc9, 0x2d, 0x18, 0xc1, 0x76, 0xde, 0xaf, 0x13, 0x10, 0xd6, 0x09, 0x5e, 0x73, 0xf8, 0xae, 0x58, - 0x73, 0x38, 0x44, 0xd9, 0x07, 0xc2, 0x9f, 0x70, 0x2b, 0x42, 0x0f, 0x4c, 0xee, 0xc2, 0x44, 0x89, - 0x43, 0xa9, 0x26, 0xcc, 0x2a, 0xf8, 0xfd, 0x25, 0xd8, 0x88, 0x4e, 0x2c, 0xe3, 0x22, 0x5c, 0xb9, - 0x0d, 0x53, 0x28, 0xfd, 0x3e, 0x0d, 0x9e, 0xf0, 0x7b, 0x6c, 0x02, 0x95, 0xb7, 0x81, 0x20, 0x69, - 0x0e, 0x6b, 0x75, 0xbf, 0xd4, 0xff, 0x0f, 0xb3, 0x48, 0xbd, 0x6f, 0x94, 0xce, 0x45, 0x7f, 0x17, - 0xe4, 0x3d, 0xd7, 0xa6, 0x6a, 0x55, 0x37, 0xca, 0xcd, 0x16, 0xbc, 0x0c, 0x31, 0xa3, 0x56, 0x45, - 0x16, 0x13, 0xfc, 0x18, 0x8d, 0x5a, 0x55, 0x3c, 0x46, 0xa3, 0x56, 0x0d, 0xd4, 0xdf, 0xa0, 0x15, - 0xea, 0xd2, 0x7e, 0xc5, 0x7f, 0x26, 0x01, 0xf0, 0x17, 0xe7, 0x4d, 0xe3, 0xc0, 0xec, 0xb9, 0x71, - 0xbe, 0x0d, 0x09, 0x3c, 0x4f, 0xad, 0x70, 0x64, 0x62, 0x6d, 0x97, 0x96, 0x87, 0x79, 0xc7, 0xcb, - 0xc1, 0x5b, 0x66, 0xa4, 0xc0, 0x43, 0x08, 0x65, 0xa4, 0x15, 0xaa, 0x3a, 0x3e, 0x69, 0x2c, 0x24, - 0xe5, 0xe0, 0x66, 0xd2, 0x10, 0xaa, 0x3c, 0x83, 0x19, 0xee, 0x6b, 0x4b, 0x53, 0xdd, 0x70, 0xf0, - 0x7b, 0x4b, 0xfc, 0x56, 0x13, 0x8d, 0xc5, 0xb3, 0x26, 0xd1, 0x3e, 0x06, 0x9b, 0x1a, 0xc8, 0x59, - 0xd5, 0x2d, 0x1d, 0xb6, 0x93, 0xfe, 0x01, 0x4c, 0x1c, 0xa8, 0x7a, 0xc5, 0x7f, 0xc3, 0xf4, 0x6f, - 0x84, 0x1c, 0x6a, 0x11, 0x25, 0xe0, 0x41, 0xcd, 0x49, 0xde, 0x69, 0xbe, 0x25, 0xe3, 0x22, 0x3c, - 0xb0, 0x37, 0x87, 0xaf, 0x5c, 0xdf, 0x94, 0xbd, 0x4d, 0xd2, 0xbb, 0xdb, 0x1b, 0x25, 0xe8, 0xc3, - 0xde, 0x04, 0xc4, 0xf3, 0x86, 0xf6, 0x58, 0xb5, 0x8f, 0xa9, 0xad, 0x7c, 0x22, 0xc1, 0x5c, 0xf4, - 0x66, 0x3c, 0xa6, 0x8e, 0xa3, 0x96, 0x29, 0xf9, 0x9f, 0xfe, 0xec, 0x7f, 0x30, 0x10, 0x7e, 0x50, - 0x88, 0x51, 0x43, 0xf3, 0x8a, 0xca, 0x24, 0x92, 0x05, 0xf2, 0xf8, 0xfd, 0xa2, 0x62, 0x8f, 0xf9, - 0x60, 0x60, 0x97, 0xe1, 0x67, 0x47, 0x61, 0x98, 0x9e, 0x50, 0xc3, 0x5d, 0x49, 0x41, 0x42, 0xf8, - 0x9a, 0x4e, 0x12, 0x30, 0xea, 0x2d, 0x93, 0x03, 0x2b, 0xd7, 0x21, 0x21, 0x7c, 0x76, 0x25, 0xe3, - 0x30, 0xb6, 0x6d, 0x6a, 0x74, 0xc7, 0xb4, 0xdd, 0xe4, 0x00, 0x5b, 0x3d, 0xa0, 0xaa, 0x56, 0x61, - 0xa8, 0xd2, 0xca, 0xa7, 0x12, 0x8c, 0xf9, 0x9f, 0x70, 0x08, 0xc0, 0xc8, 0x3b, 0xfb, 0xf9, 0xfd, - 0xfc, 0x46, 0x72, 0x80, 0x31, 0xdc, 0xc9, 0x6f, 0x6f, 0x6c, 0x6e, 0xdf, 0x4f, 0x4a, 0x6c, 0xb1, - 0xbb, 0xbf, 0xbd, 0xcd, 0x16, 0x83, 0x64, 0x02, 0xe2, 0x7b, 0xfb, 0xb9, 0x5c, 0x3e, 0xbf, 0x91, - 0xdf, 0x48, 0xc6, 0x18, 0xd1, 0xbd, 0xf5, 0xcd, 0x47, 0xf9, 0x8d, 0xe4, 0x10, 0xc3, 0xdb, 0xdf, - 0x7e, 0xb8, 0xfd, 0xe4, 0xbd, 0xed, 0xe4, 0x30, 0xc7, 0xcb, 0x3e, 0xde, 0x7c, 0xfa, 0x34, 0xbf, - 0x91, 0x1c, 0x61, 0x78, 0x8f, 0xf2, 0xeb, 0x7b, 0xf9, 0x8d, 0xe4, 0x28, 0xdb, 0xda, 0xd9, 0xcd, - 0xe7, 0x1f, 0xef, 0xb0, 0xad, 0x31, 0xb6, 0xcc, 0xad, 0x6f, 0xe7, 0xf2, 0x8f, 0x18, 0x97, 0x38, - 0xd3, 0x70, 0x37, 0xbf, 0x95, 0xcf, 0xb1, 0x4d, 0x58, 0xfb, 0xc3, 0x10, 0x8c, 0xa3, 0x43, 0xfd, - 0xc7, 0xc1, 0x37, 0x21, 0xc1, 0x4f, 0x95, 0xcf, 0xd7, 0x82, 0xcb, 0x53, 0xf3, 0x2d, 0xcf, 0xb6, - 0x79, 0xe6, 0x3c, 0x65, 0x80, 0xdc, 0x85, 0x71, 0x81, 0xc8, 0x21, 0x93, 0x21, 0x15, 0x2b, 0x22, - 0xa9, 0x97, 0x70, 0xdd, 0x29, 0xd0, 0x94, 0x01, 0x26, 0x95, 0xdf, 0x9d, 0x3e, 0xa5, 0x0a, 0x44, - 0xdd, 0xa5, 0x46, 0x6f, 0xa7, 0x32, 0x40, 0xbe, 0x05, 0x09, 0x9e, 0x4b, 0xb9, 0xd4, 0x4b, 0x21, - 0x7d, 0x24, 0xc5, 0x9e, 0xa1, 0x42, 0x06, 0xc6, 0xee, 0x53, 0x97, 0x93, 0xcf, 0x86, 0xe4, 0x61, - 0x66, 0x4f, 0x09, 0xa6, 0x28, 0x03, 0x64, 0x0b, 0xe2, 0x3e, 0xbe, 0x43, 0xb8, 0x7e, 0x9d, 0x6a, - 0x42, 0x2a, 0xd5, 0x66, 0xdb, 0xbb, 0x18, 0xca, 0xc0, 0x1b, 0x12, 0xd3, 0x9e, 0x17, 0xb2, 0x16, - 0xed, 0x23, 0xf5, 0xed, 0x0c, 0xed, 0x37, 0x60, 0xc2, 0x2f, 0x66, 0x9c, 0xc7, 0x82, 0x90, 0xca, - 0xa2, 0x55, 0xae, 0x33, 0x97, 0xb5, 0x9f, 0xc4, 0x61, 0x84, 0xbf, 0xea, 0x90, 0x77, 0x01, 0xf8, - 0x2f, 0xcc, 0xff, 0x73, 0x6d, 0x3f, 0xdd, 0xa7, 0xe6, 0xdb, 0x3f, 0x05, 0x29, 0x0b, 0x3f, 0xfa, - 0xd3, 0xdf, 0x7e, 0x36, 0x38, 0xa3, 0x4c, 0xae, 0x9e, 0xdc, 0x5c, 0x3d, 0x32, 0x8b, 0xde, 0x3f, - 0x0d, 0xef, 0x48, 0x2b, 0xe4, 0x3d, 0x00, 0xde, 0x4a, 0x44, 0xf9, 0x46, 0xbe, 0x10, 0xa7, 0xb8, - 0x03, 0x5a, 0x5b, 0x8e, 0x56, 0xc6, 0xbc, 0x9f, 0x60, 0x8c, 0xbf, 0x0b, 0xe3, 0x01, 0xe3, 0x3d, - 0xea, 0x12, 0x59, 0xf8, 0xe8, 0x1b, 0xe5, 0xde, 0xc9, 0xfe, 0x2b, 0xc8, 0x7c, 0x5e, 0x99, 0xf6, - 0x98, 0x3b, 0xd4, 0x15, 0xf8, 0x1b, 0x90, 0x14, 0x1f, 0x20, 0x51, 0xfd, 0xcb, 0xed, 0x9f, 0x26, - 0xb9, 0x98, 0x2b, 0x67, 0xbd, 0x5b, 0x2a, 0x69, 0x14, 0xb6, 0xa0, 0xcc, 0xfa, 0x96, 0x08, 0x6f, - 0x90, 0x94, 0xc9, 0xfb, 0x00, 0x12, 0xde, 0xdf, 0x1b, 0x50, 0x54, 0xe0, 0xea, 0xe8, 0x7f, 0x1e, - 0x3a, 0x1a, 0x93, 0x42, 0xfe, 0xb3, 0xca, 0x94, 0xcf, 0xdf, 0xe2, 0x74, 0x8c, 0xf5, 0xfd, 0xfe, - 0x13, 0xc3, 0x2c, 0xb2, 0x9b, 0x54, 0xe2, 0x8c, 0x1d, 0x26, 0x66, 0xc6, 0xa8, 0xf4, 0x7c, 0xc9, - 0xe2, 0x15, 0x64, 0xba, 0xa8, 0x2c, 0x30, 0xa6, 0x45, 0x86, 0x45, 0xb5, 0x55, 0xfe, 0x75, 0xc8, - 0xab, 0x53, 0x4c, 0xc8, 0x76, 0xff, 0x09, 0xe5, 0x32, 0x32, 0x9e, 0x4b, 0x25, 0x03, 0x6d, 0x57, - 0x7f, 0xc8, 0x5a, 0xa0, 0x8f, 0x3c, 0xa5, 0x9f, 0x27, 0xd7, 0x78, 0x4a, 0xa7, 0x22, 0x4a, 0xd7, - 0x10, 0x47, 0x50, 0xfa, 0xfd, 0xe7, 0xcc, 0x47, 0x32, 0x4a, 0x21, 0x2b, 0x2d, 0x16, 0x90, 0x7b, - 0x7d, 0xe5, 0x29, 0x8f, 0x0f, 0x69, 0xe5, 0xa3, 0xbd, 0xa0, 0xfc, 0xe5, 0x05, 0x1a, 0x21, 0xa2, - 0x3f, 0xb8, 0x23, 0xde, 0x90, 0xc8, 0x1d, 0x18, 0x79, 0x80, 0x7f, 0xd0, 0x25, 0x1d, 0x2c, 0x4d, - 0xf1, 0x7b, 0xca, 0x91, 0x72, 0x87, 0xb4, 0x74, 0x1c, 0xf4, 0x20, 0xef, 0xff, 0xf1, 0x8b, 0x45, - 0xe9, 0xf3, 0x2f, 0x16, 0xa5, 0xbf, 0x7e, 0xb1, 0x28, 0x7d, 0xf2, 0xe5, 0xe2, 0xc0, 0xe7, 0x5f, - 0x2e, 0x0e, 0xfc, 0xf9, 0xcb, 0xc5, 0x81, 0x6f, 0xbf, 0x56, 0xd6, 0xdd, 0xc3, 0x5a, 0x31, 0x53, - 0x32, 0xab, 0xab, 0xaa, 0x5d, 0x55, 0x35, 0xd5, 0xb2, 0xcd, 0x23, 0x5a, 0x72, 0xbd, 0xd5, 0xaa, - 0xf7, 0xe7, 0xe0, 0xcf, 0x06, 0x67, 0xd7, 0x11, 0xb0, 0xc3, 0xb7, 0x33, 0x9b, 0x66, 0x66, 0xdd, - 0xd2, 0x8b, 0x23, 0xa8, 0xc3, 0x9b, 0xff, 0x0e, 0x00, 0x00, 0xff, 0xff, 0x0b, 0x6e, 0x2e, 0x1c, - 0x0a, 0x2d, 0x00, 0x00, + // 3061 bytes of a gzipped FileDescriptorProto + 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xec, 0x3a, 0x3b, 0x70, 0x1b, 0xd7, + 0xb5, 0x5c, 0x82, 0x3f, 0x1c, 0xf0, 0x03, 0x5e, 0x7e, 0xb4, 0x84, 0x64, 0x82, 0x5a, 0xdb, 0x32, + 0x45, 0xeb, 0x81, 0x16, 0xfd, 0x3c, 0x4f, 0xd2, 0xf3, 0x7b, 0x0a, 0x01, 0x42, 0x12, 0x29, 0x89, + 0xa2, 0x49, 0xd1, 0x9f, 0x4c, 0x26, 0xc8, 0x02, 0x7b, 0x09, 0x2e, 0x09, 0xec, 0xae, 0x77, 0x17, + 0x54, 0x98, 0x8c, 0x9b, 0x4c, 0x66, 0x5c, 0xa4, 0xf1, 0x24, 0x65, 0x66, 0xe2, 0x14, 0xa9, 0x9c, + 0x3a, 0x4d, 0x26, 0x45, 0xca, 0x94, 0xce, 0xa4, 0x49, 0x1a, 0x4c, 0xc6, 0x4e, 0xe2, 0x19, 0x74, + 0xe9, 0x53, 0x64, 0xee, 0xb9, 0xfb, 0xb9, 0x8b, 0x0f, 0x01, 0x50, 0xa2, 0xdd, 0xa4, 0xc3, 0x3d, + 0xf7, 0xfc, 0xef, 0xb9, 0xe7, 0x73, 0x17, 0x30, 0x6b, 0x1d, 0x97, 0x57, 0x55, 0x4b, 0x5f, 0x75, + 0x6a, 0xc5, 0xaa, 0xee, 0x66, 0x2c, 0xdb, 0x74, 0x4d, 0x12, 0x53, 0x2d, 0x3d, 0x75, 0xb9, 0x6c, + 0x9a, 0xe5, 0x0a, 0x5d, 0x45, 0x50, 0xb1, 0x76, 0xb0, 0x4a, 0xab, 0x96, 0x7b, 0xca, 0x31, 0x52, + 0xe9, 0xe6, 0x4d, 0x57, 0xaf, 0x52, 0xc7, 0x55, 0xab, 0x96, 0x87, 0xa0, 0x1c, 0xdf, 0x72, 0x32, + 0xba, 0x89, 0xbc, 0x4b, 0xa6, 0x4d, 0x57, 0x4f, 0x6e, 0xae, 0x96, 0xa9, 0x41, 0x6d, 0xd5, 0xa5, + 0x9a, 0x87, 0xb3, 0x2c, 0xe0, 0x18, 0xd4, 0x7d, 0x66, 0xda, 0xc7, 0xba, 0x51, 0x6e, 0x87, 0x79, + 0xc5, 0x13, 0xc7, 0x30, 0x55, 0xc3, 0x30, 0x5d, 0xd5, 0xd5, 0x4d, 0xc3, 0xf1, 0x76, 0x03, 0x23, + 0x0e, 0xa9, 0x5a, 0x71, 0x0f, 0x39, 0x54, 0xf9, 0x34, 0x0e, 0xb3, 0x5b, 0x66, 0x71, 0x0f, 0x0d, + 0xdb, 0xa5, 0x1f, 0xd6, 0xa8, 0xe3, 0x6e, 0xba, 0xb4, 0x4a, 0xd6, 0x60, 0xcc, 0xb2, 0x75, 0xd3, + 0xd6, 0xdd, 0x53, 0x59, 0x5a, 0x92, 0x96, 0xa5, 0xec, 0x7c, 0xa3, 0x9e, 0x26, 0x3e, 0xec, 0x86, + 0x59, 0xd5, 0x5d, 0xb4, 0x75, 0x37, 0xc0, 0x23, 0x6f, 0x41, 0xdc, 0x50, 0xab, 0xd4, 0xb1, 0xd4, + 0x12, 0x95, 0x63, 0x4b, 0xd2, 0x72, 0x3c, 0x7b, 0xa9, 0x51, 0x4f, 0xcf, 0x04, 0x40, 0x81, 0x2a, + 0xc4, 0x24, 0x6f, 0x42, 0xbc, 0x54, 0xd1, 0xa9, 0xe1, 0x16, 0x74, 0x4d, 0x1e, 0x43, 0x32, 0x94, + 0xc5, 0x81, 0x9b, 0x9a, 0x28, 0xcb, 0x87, 0x91, 0x3d, 0x18, 0xa9, 0xa8, 0x45, 0x5a, 0x71, 0xe4, + 0xa1, 0xa5, 0xd8, 0x72, 0x62, 0xed, 0xd5, 0x8c, 0x6a, 0xe9, 0x99, 0x76, 0xa6, 0x64, 0x1e, 0x21, + 0x5e, 0xde, 0x70, 0xed, 0xd3, 0xec, 0x6c, 0xa3, 0x9e, 0x4e, 0x72, 0x42, 0x81, 0xad, 0xc7, 0x8a, + 0x94, 0x21, 0x21, 0x38, 0x4e, 0x1e, 0x46, 0xce, 0x2b, 0x9d, 0x39, 0xaf, 0x87, 0xc8, 0x9c, 0xfd, + 0x42, 0xa3, 0x9e, 0x9e, 0x13, 0x58, 0x08, 0x32, 0x44, 0xce, 0xe4, 0x63, 0x09, 0x66, 0x6d, 0xfa, + 0x61, 0x4d, 0xb7, 0xa9, 0x56, 0x30, 0x4c, 0x8d, 0x16, 0x3c, 0x63, 0x46, 0x50, 0xe4, 0xcd, 0xce, + 0x22, 0x77, 0x3d, 0xaa, 0x6d, 0x53, 0xa3, 0xa2, 0x61, 0x4a, 0xa3, 0x9e, 0xbe, 0x62, 0xb7, 0x6c, + 0x86, 0x0a, 0xc8, 0xd2, 0x2e, 0x69, 0xdd, 0x27, 0x4f, 0x60, 0xcc, 0x32, 0xb5, 0x82, 0x63, 0xd1, + 0x92, 0x3c, 0xb8, 0x24, 0x2d, 0x27, 0xd6, 0x2e, 0x67, 0x78, 0xc4, 0xa1, 0x0e, 0x2c, 0x2a, 0x33, + 0x27, 0x37, 0x33, 0x3b, 0xa6, 0xb6, 0x67, 0xd1, 0x12, 0x9e, 0xe7, 0xb4, 0xc5, 0x17, 0x11, 0xde, + 0xa3, 0x1e, 0x90, 0xec, 0x40, 0xdc, 0x67, 0xe8, 0xc8, 0xa3, 0x68, 0xce, 0x99, 0x1c, 0x79, 0x58, + 0xf1, 0x85, 0x13, 0x09, 0x2b, 0x0f, 0x46, 0x72, 0x30, 0xaa, 0x1b, 0x65, 0x9b, 0x3a, 0x8e, 0x1c, + 0x47, 0x7e, 0x04, 0x19, 0x6d, 0x72, 0x58, 0xce, 0x34, 0x0e, 0xf4, 0x72, 0x76, 0x8e, 0x29, 0xe6, + 0xa1, 0x09, 0x5c, 0x7c, 0x4a, 0x72, 0x0f, 0xc6, 0x1c, 0x6a, 0x9f, 0xe8, 0x25, 0xea, 0xc8, 0x20, + 0x70, 0xd9, 0xe3, 0x40, 0x8f, 0x0b, 0x2a, 0xe3, 0xe3, 0x89, 0xca, 0xf8, 0x30, 0x16, 0xe3, 0x4e, + 0xe9, 0x90, 0x6a, 0xb5, 0x0a, 0xb5, 0xe5, 0x44, 0x18, 0xe3, 0x01, 0x50, 0x8c, 0xf1, 0x00, 0x98, + 0x52, 0x21, 0x21, 0x9c, 0x16, 0x79, 0x19, 0x62, 0xc7, 0x94, 0x5f, 0xac, 0x78, 0x76, 0xba, 0x51, + 0x4f, 0x4f, 0x1c, 0x53, 0xf1, 0x4e, 0xb1, 0x5d, 0x72, 0x1d, 0x86, 0x4f, 0xd4, 0x4a, 0x8d, 0xe2, + 0xb9, 0xc4, 0xb3, 0x33, 0x8d, 0x7a, 0x7a, 0x0a, 0x01, 0x02, 0x22, 0xc7, 0xb8, 0x33, 0x78, 0x4b, + 0x4a, 0x1d, 0x40, 0xb2, 0x39, 0x1e, 0x2f, 0x44, 0x4e, 0x15, 0x2e, 0x75, 0x08, 0xc2, 0x8b, 0x10, + 0xa7, 0xfc, 0x33, 0x06, 0x13, 0x91, 0xa3, 0x26, 0x77, 0x60, 0xc8, 0x3d, 0xb5, 0x28, 0x8a, 0x99, + 0x5c, 0x4b, 0x8a, 0xc1, 0xf0, 0xf4, 0xd4, 0xa2, 0x78, 0xc7, 0x27, 0x19, 0x46, 0x24, 0x40, 0x91, + 0x86, 0x09, 0xb7, 0x4c, 0xdb, 0x75, 0xe4, 0xc1, 0xa5, 0xd8, 0xf2, 0x04, 0x17, 0x8e, 0x00, 0x51, + 0x38, 0x02, 0xc8, 0xf7, 0xa2, 0xc9, 0x20, 0x86, 0x41, 0xf3, 0x72, 0x6b, 0xe8, 0x9d, 0x3f, 0x0b, + 0xdc, 0x86, 0x84, 0x5b, 0x71, 0x0a, 0xd4, 0x50, 0x8b, 0x15, 0xaa, 0xc9, 0x43, 0x4b, 0xd2, 0xf2, + 0x58, 0x56, 0x6e, 0xd4, 0xd3, 0xb3, 0x2e, 0xf3, 0x28, 0x42, 0x05, 0x5a, 0x08, 0xa1, 0x98, 0x33, + 0xa9, 0xed, 0x16, 0x58, 0x16, 0x95, 0x87, 0x85, 0x9c, 0x49, 0x6d, 0x77, 0x5b, 0xad, 0xd2, 0x48, + 0xce, 0xf4, 0x60, 0xe4, 0x2e, 0x4c, 0xd4, 0x1c, 0x5a, 0x28, 0x55, 0x6a, 0x8e, 0x4b, 0xed, 0xcd, + 0x1d, 0x79, 0x04, 0x25, 0xa6, 0x1a, 0xf5, 0xf4, 0x7c, 0xcd, 0xa1, 0x39, 0x1f, 0x2e, 0x10, 0x8f, + 0x8b, 0xf0, 0xaf, 0x2b, 0xc4, 0x14, 0x17, 0x26, 0x22, 0xf7, 0x92, 0xdc, 0x6a, 0x73, 0xe4, 0x1e, + 0x06, 0x1e, 0x39, 0x69, 0x3d, 0xf2, 0xbe, 0x0f, 0x5c, 0xf9, 0x8b, 0x04, 0xc9, 0xe6, 0x9c, 0xcb, + 0xe8, 0x3f, 0xac, 0xd1, 0x1a, 0xf5, 0x0c, 0x44, 0x7a, 0x04, 0x88, 0xf4, 0x08, 0x20, 0xff, 0x0d, + 0x70, 0x64, 0x16, 0x0b, 0x0e, 0xc5, 0x42, 0x36, 0x18, 0x1e, 0xca, 0x91, 0x59, 0xdc, 0xa3, 0x4d, + 0x85, 0xcc, 0x87, 0x11, 0x0d, 0xa6, 0x19, 0x95, 0xcd, 0xe5, 0x15, 0x18, 0x82, 0x1f, 0x6c, 0x0b, + 0x1d, 0xcb, 0x40, 0xf6, 0xa5, 0x46, 0x3d, 0xbd, 0x70, 0x64, 0x16, 0x05, 0x98, 0x68, 0xd1, 0x54, + 0xd3, 0x96, 0xf2, 0x47, 0x09, 0xa6, 0xb7, 0xcc, 0xe2, 0x8e, 0x4d, 0x19, 0xc2, 0xd7, 0x66, 0xdc, + 0x7f, 0xc1, 0x28, 0xa3, 0xd2, 0x35, 0x6e, 0x52, 0x9c, 0xd7, 0xdf, 0x23, 0xb3, 0xb8, 0xa9, 0x45, + 0xea, 0x2f, 0x87, 0x90, 0x1b, 0x30, 0x62, 0x53, 0xd5, 0x31, 0x0d, 0xbc, 0x0b, 0x1e, 0x36, 0x87, + 0x88, 0xd8, 0x1c, 0xa2, 0xfc, 0x8b, 0x9f, 0x57, 0x4e, 0x35, 0x4a, 0xb4, 0xe2, 0x9b, 0xb4, 0x02, + 0x23, 0x5c, 0xa2, 0x68, 0x13, 0xb2, 0x17, 0x6d, 0x42, 0xc0, 0x39, 0x6d, 0x0a, 0x9c, 0x16, 0xeb, + 0xea, 0x34, 0xc1, 0xfc, 0xa1, 0xbe, 0xcc, 0x1f, 0xee, 0xc1, 0xfc, 0xbf, 0x4b, 0x30, 0xb3, 0x85, + 0x4a, 0x45, 0x3d, 0x10, 0xb5, 0x4a, 0xea, 0xd7, 0xaa, 0xc1, 0xae, 0x56, 0xdd, 0x85, 0x91, 0x03, + 0xbd, 0xe2, 0x52, 0x1b, 0x3d, 0x90, 0x58, 0x9b, 0x0e, 0xc2, 0x94, 0xba, 0xf7, 0x70, 0x83, 0x6b, + 0xce, 0x91, 0x44, 0xcd, 0x39, 0xa4, 0xcf, 0x63, 0x7e, 0x08, 0xe3, 0x22, 0x6f, 0xf2, 0xbf, 0x30, + 0xe2, 0xb8, 0xaa, 0x4b, 0x1d, 0x59, 0x5a, 0x8a, 0x2d, 0x4f, 0xae, 0x4d, 0x04, 0xe2, 0x19, 0x94, + 0x33, 0xe3, 0x08, 0x22, 0x33, 0x0e, 0x51, 0xbe, 0x9a, 0x82, 0xd8, 0x96, 0x59, 0x24, 0x4b, 0x30, + 0x18, 0x38, 0x27, 0xd9, 0xa8, 0xa7, 0xc7, 0x75, 0xd1, 0x2d, 0x83, 0xba, 0x16, 0xed, 0x4a, 0x27, + 0x7a, 0xec, 0x4a, 0x2f, 0x3c, 0xa2, 0x22, 0x2d, 0xf6, 0x68, 0xcf, 0x2d, 0x76, 0x36, 0xe8, 0x96, + 0x79, 0x07, 0x35, 0xeb, 0xfb, 0xac, 0x8f, 0xe6, 0xf8, 0xdd, 0x68, 0x3d, 0x84, 0x68, 0x8a, 0x3a, + 0x7f, 0x15, 0x3c, 0xe9, 0xd0, 0x0a, 0x27, 0x50, 0xc0, 0x52, 0x20, 0xe0, 0x45, 0x77, 0xbe, 0xd7, + 0x61, 0xd8, 0x7c, 0x66, 0x50, 0xdb, 0x1b, 0x39, 0xd0, 0xeb, 0x08, 0x10, 0xbd, 0x8e, 0x00, 0x42, + 0xe1, 0x32, 0xba, 0xbf, 0x80, 0x4b, 0xe7, 0x50, 0xb7, 0x0a, 0x35, 0x87, 0xda, 0x85, 0xb2, 0x6d, + 0xd6, 0x2c, 0x47, 0x9e, 0xc2, 0xbb, 0x7d, 0xad, 0x51, 0x4f, 0x2b, 0x88, 0xf6, 0xc4, 0xc7, 0xda, + 0x77, 0xa8, 0x7d, 0x1f, 0x71, 0x04, 0x9e, 0x72, 0x27, 0x1c, 0xf2, 0x63, 0x09, 0xae, 0x95, 0xcc, + 0xaa, 0xc5, 0x7a, 0x0b, 0xaa, 0x15, 0xce, 0x12, 0x39, 0xb3, 0x24, 0x2d, 0x8f, 0x67, 0xdf, 0x68, + 0xd4, 0xd3, 0x37, 0x42, 0x8a, 0x77, 0xba, 0x0b, 0x57, 0xba, 0x63, 0x47, 0x46, 0xbf, 0xa1, 0x1e, + 0x47, 0x3f, 0x71, 0x8c, 0x18, 0x7e, 0xe1, 0x63, 0xc4, 0xf8, 0x8b, 0x18, 0x23, 0x7e, 0x29, 0xc1, + 0x92, 0xd7, 0x90, 0xeb, 0x46, 0xb9, 0x60, 0x53, 0xc7, 0xac, 0xd9, 0x25, 0x5a, 0xf0, 0x42, 0xa3, + 0x4a, 0x0d, 0xd7, 0x91, 0xe7, 0x50, 0xf7, 0xe5, 0x76, 0x92, 0x76, 0x3d, 0x82, 0x5d, 0x01, 0x3f, + 0x7b, 0xa3, 0x51, 0x4f, 0x2f, 0x87, 0x5c, 0xdb, 0xe1, 0x08, 0xca, 0x2c, 0x9e, 0x8d, 0x49, 0x1e, + 0xc2, 0x68, 0xc9, 0xa6, 0x6c, 0xa4, 0xc7, 0xd6, 0x2c, 0xb1, 0x96, 0xca, 0xf0, 0x99, 0x3e, 0xe3, + 0x3f, 0x21, 0x64, 0x9e, 0xfa, 0x4f, 0x08, 0x7c, 0xe2, 0xf1, 0xd0, 0xc5, 0x89, 0xc7, 0x03, 0x89, + 0x63, 0xd3, 0xe4, 0x0b, 0x19, 0x9b, 0x92, 0xcf, 0x31, 0x36, 0x7d, 0x07, 0x12, 0xc7, 0xb7, 0x9c, + 0x82, 0xaf, 0xd0, 0x34, 0xb2, 0xba, 0x2a, 0xba, 0x39, 0x7c, 0xdb, 0x60, 0xce, 0xf6, 0xb4, 0xe4, + 0xdd, 0xf0, 0xf1, 0x2d, 0x67, 0xb3, 0x45, 0x45, 0x08, 0xa1, 0x2c, 0x35, 0x31, 0xee, 0x9e, 0x34, + 0x99, 0x74, 0x0e, 0x17, 0x4f, 0xef, 0x80, 0xaf, 0xb7, 0x6e, 0xe2, 0xeb, 0x41, 0xa3, 0xc3, 0xde, + 0x6c, 0xaf, 0xc3, 0x1e, 0xd9, 0x84, 0x69, 0x7e, 0x77, 0x5d, 0xb7, 0x52, 0x70, 0x68, 0xc9, 0x34, + 0x34, 0x47, 0x9e, 0x5f, 0x92, 0x96, 0x63, 0xbc, 0x6f, 0xc3, 0xcd, 0xa7, 0x6e, 0x65, 0x8f, 0x6f, + 0x89, 0x7d, 0x5b, 0xd3, 0xd6, 0x7f, 0xe6, 0xc6, 0x73, 0xcf, 0x10, 0xff, 0x90, 0x60, 0x7e, 0x8b, + 0x75, 0xc1, 0x5e, 0x8e, 0xd2, 0x7f, 0x40, 0xfd, 0x0e, 0x49, 0x68, 0xcb, 0xa4, 0x1e, 0xda, 0xb2, + 0x0b, 0x2f, 0xea, 0x6f, 0xc3, 0xb8, 0x41, 0x9f, 0x15, 0x9a, 0x92, 0x2e, 0xd6, 0x4f, 0x83, 0x3e, + 0xdb, 0x69, 0xcd, 0xbb, 0x09, 0x01, 0xac, 0xfc, 0x7a, 0x10, 0x2e, 0xb5, 0x18, 0xea, 0x58, 0xa6, + 0xe1, 0x50, 0xf2, 0x73, 0x09, 0x64, 0x3b, 0xdc, 0xc0, 0x23, 0x66, 0x99, 0xaf, 0x56, 0x71, 0xb9, + 0xed, 0x89, 0xb5, 0xdb, 0x7e, 0x81, 0x6d, 0xc7, 0x20, 0xb3, 0xdb, 0x44, 0xbc, 0xcb, 0x69, 0x79, + 0xe5, 0x7d, 0xb5, 0x51, 0x4f, 0x5f, 0xb5, 0xdb, 0x63, 0x08, 0xda, 0x5e, 0xea, 0x80, 0x92, 0xb2, + 0xe1, 0xca, 0x59, 0xfc, 0x2f, 0x24, 0x2c, 0x0c, 0x98, 0x13, 0x06, 0x2a, 0x6e, 0x25, 0x3e, 0x78, + 0xf6, 0x33, 0x38, 0x5c, 0x87, 0x61, 0x6a, 0xdb, 0xa6, 0x2d, 0xca, 0x44, 0x80, 0x88, 0x8a, 0x00, + 0xe5, 0x23, 0x9c, 0xbb, 0xa2, 0xf2, 0xc8, 0x21, 0x10, 0x3e, 0xf3, 0xf1, 0xb5, 0x37, 0xf4, 0xf1, + 0xf3, 0x48, 0x35, 0x0f, 0x7d, 0xa1, 0x8e, 0xd9, 0xc5, 0x46, 0x3d, 0x9d, 0xc2, 0xd1, 0x2e, 0x04, + 0x8a, 0x9e, 0x4e, 0x36, 0xef, 0x29, 0x1f, 0x27, 0x60, 0x18, 0x0b, 0x3d, 0xb9, 0x06, 0x43, 0xf8, + 0x58, 0xc0, 0xad, 0xc3, 0x81, 0xd9, 0x88, 0x3e, 0x14, 0xe0, 0x3e, 0xc9, 0xc3, 0x94, 0x1f, 0x88, + 0x85, 0x03, 0xb5, 0xe4, 0x7a, 0x56, 0x4a, 0xd9, 0x2b, 0x8d, 0x7a, 0x5a, 0xf6, 0xb7, 0xee, 0xe1, + 0x8e, 0x40, 0x3c, 0x19, 0xdd, 0x21, 0xb7, 0x21, 0x81, 0xfd, 0x0a, 0x6f, 0x5f, 0xbc, 0xe9, 0x0f, + 0xb3, 0x2e, 0x03, 0xf3, 0xb6, 0x43, 0xcc, 0xba, 0x21, 0x94, 0x5d, 0x07, 0xec, 0x72, 0x7c, 0x5a, + 0x3e, 0x3a, 0xe1, 0x75, 0x40, 0x78, 0x0b, 0x71, 0x42, 0x00, 0x93, 0x32, 0x4c, 0x05, 0xa5, 0xbd, + 0xa2, 0x57, 0x75, 0xd7, 0x7f, 0xc7, 0x5d, 0x44, 0xc7, 0xa2, 0x33, 0x82, 0x5a, 0xfe, 0x08, 0x11, + 0x78, 0x34, 0x33, 0xe7, 0xca, 0x76, 0x64, 0x23, 0xd2, 0x9a, 0x4c, 0x46, 0xf7, 0xc8, 0x6f, 0x24, + 0xb8, 0xd6, 0x24, 0xa9, 0x50, 0x3c, 0x0d, 0x6e, 0x71, 0xa1, 0x54, 0x51, 0x1d, 0x87, 0x3f, 0xd0, + 0x8c, 0x0a, 0xaf, 0xba, 0xed, 0x14, 0xc8, 0x9e, 0xfa, 0xb7, 0x39, 0xc7, 0x88, 0xb6, 0xd5, 0x2a, + 0xe5, 0x3a, 0xad, 0x36, 0xea, 0xe9, 0xd7, 0xed, 0x6e, 0xb8, 0x82, 0x2b, 0xae, 0x76, 0x45, 0x26, + 0x7b, 0x90, 0xb0, 0xa8, 0x5d, 0xd5, 0x1d, 0x07, 0xfb, 0x78, 0xfe, 0xe2, 0x3c, 0x2f, 0xe8, 0xb6, + 0x13, 0xee, 0x72, 0xaf, 0x0b, 0xe8, 0xa2, 0xd7, 0x05, 0x30, 0xeb, 0x19, 0x4b, 0xa6, 0xad, 0x99, + 0x06, 0xe5, 0x4f, 0xf8, 0x63, 0xde, 0xb0, 0xe4, 0xc1, 0x22, 0xc3, 0x92, 0x07, 0x23, 0x8f, 0x61, + 0x9a, 0xb7, 0xfa, 0x05, 0x8d, 0x5a, 0x36, 0x2d, 0x61, 0xdf, 0x13, 0xc7, 0xc3, 0x5e, 0x62, 0x81, + 0xce, 0x37, 0x37, 0x82, 0xbd, 0xc8, 0x69, 0x24, 0x9b, 0x77, 0xc9, 0x46, 0x30, 0xe3, 0x40, 0x8b, + 0x49, 0x3d, 0x4f, 0x39, 0xa9, 0xaf, 0x24, 0x48, 0x08, 0x0e, 0x20, 0xbb, 0x30, 0xe6, 0xd4, 0x8a, + 0x47, 0xb4, 0x14, 0x24, 0xcc, 0xc5, 0xf6, 0xae, 0xca, 0xec, 0x71, 0x34, 0xaf, 0x19, 0xf2, 0x68, + 0x22, 0xcd, 0x90, 0x07, 0xc3, 0x94, 0x45, 0xed, 0x22, 0x7f, 0x93, 0xf2, 0x53, 0x16, 0x03, 0x44, + 0x52, 0x16, 0x03, 0xa4, 0x3e, 0x80, 0x51, 0x8f, 0x2f, 0xbb, 0xc0, 0xc7, 0xba, 0xa1, 0x89, 0x17, + 0x98, 0xad, 0xc5, 0x0b, 0xcc, 0xd6, 0xc1, 0x45, 0x1f, 0x3c, 0xfb, 0xa2, 0xa7, 0x74, 0x98, 0x69, + 0x73, 0x0d, 0xce, 0x91, 0x74, 0xa5, 0xae, 0xa5, 0xff, 0x17, 0x12, 0x5c, 0xeb, 0x2d, 0xe2, 0x7b, + 0x13, 0xff, 0x50, 0x14, 0xef, 0xcf, 0x88, 0x11, 0x86, 0x4d, 0xd2, 0xba, 0x29, 0x78, 0xf1, 0x6d, + 0x96, 0xf2, 0xd3, 0x61, 0xb8, 0x7c, 0x86, 0x8a, 0x6c, 0x3c, 0x59, 0xa8, 0xaa, 0xdf, 0xd7, 0xab, + 0xb5, 0x6a, 0x38, 0x9b, 0x1c, 0xd8, 0x6a, 0x89, 0x95, 0x45, 0x2f, 0xf4, 0xfe, 0xaf, 0x9b, 0xa1, + 0x99, 0xc7, 0x9c, 0x83, 0x0f, 0xbd, 0xe7, 0xd1, 0x0b, 0xf5, 0xba, 0xda, 0x1e, 0x43, 0xac, 0xd7, + 0x1d, 0x50, 0xc8, 0x6f, 0x25, 0xb8, 0xda, 0x51, 0x45, 0xcc, 0x7d, 0xa6, 0x59, 0xc1, 0xa0, 0x4e, + 0xac, 0xe5, 0xce, 0xab, 0x6a, 0xf6, 0x74, 0xc7, 0x34, 0x2b, 0x5c, 0xe1, 0xd7, 0x1b, 0xf5, 0xf4, + 0x6b, 0xd5, 0xb3, 0xf0, 0x04, 0xb5, 0x5f, 0x3a, 0x13, 0x91, 0x35, 0x1b, 0x67, 0x39, 0xe7, 0xa2, + 0xe2, 0x5e, 0xe9, 0x6e, 0x66, 0x6f, 0xa2, 0x9f, 0x44, 0x63, 0xfe, 0x95, 0x56, 0xff, 0x32, 0x86, + 0xfd, 0xc5, 0xbd, 0xf2, 0xbb, 0x41, 0x48, 0x77, 0xe1, 0x41, 0x7e, 0xd5, 0x43, 0x60, 0xae, 0xf7, + 0xa2, 0xcd, 0x85, 0x06, 0xe7, 0x37, 0x71, 0xbe, 0x4a, 0x1e, 0xe2, 0x58, 0x07, 0x1e, 0xe9, 0x8e, + 0x4b, 0x6e, 0xc1, 0x08, 0xb6, 0xf3, 0x7e, 0x9d, 0x80, 0xb0, 0x4e, 0xf0, 0x9a, 0xc3, 0x77, 0xc5, + 0x9a, 0xc3, 0x21, 0xca, 0x3e, 0x10, 0xfe, 0x84, 0x5b, 0x11, 0x7a, 0x60, 0x72, 0x17, 0x26, 0x4a, + 0x1c, 0x4a, 0x35, 0x61, 0x56, 0xc1, 0xaf, 0x35, 0xc1, 0x46, 0x74, 0x62, 0x19, 0x17, 0xe1, 0xca, + 0x6d, 0x98, 0x42, 0xe9, 0xf7, 0x69, 0xf0, 0xe0, 0xdf, 0x63, 0x13, 0xa8, 0xbc, 0x0d, 0x04, 0x49, + 0x73, 0x58, 0xab, 0xfb, 0xa5, 0xfe, 0x7f, 0x98, 0x45, 0xea, 0x7d, 0xa3, 0x74, 0x2e, 0xfa, 0xbb, + 0x20, 0xef, 0xb9, 0x36, 0x55, 0xab, 0xba, 0x51, 0x6e, 0xb6, 0xe0, 0x65, 0x88, 0x19, 0xb5, 0x2a, + 0xb2, 0x98, 0xe0, 0xc7, 0x68, 0xd4, 0xaa, 0xe2, 0x31, 0x1a, 0xb5, 0x6a, 0xa0, 0xfe, 0x06, 0xad, + 0x50, 0x97, 0xf6, 0x2b, 0xfe, 0x33, 0x09, 0x80, 0xbf, 0x38, 0x6f, 0x1a, 0x07, 0x66, 0xcf, 0x8d, + 0xf3, 0x6d, 0x48, 0xe0, 0x79, 0x6a, 0x85, 0x23, 0x13, 0x6b, 0xbb, 0xb4, 0x3c, 0xcc, 0x3b, 0x5e, + 0x0e, 0xde, 0x32, 0x23, 0x05, 0x1e, 0x42, 0x28, 0x23, 0xad, 0x50, 0xd5, 0xf1, 0x49, 0x63, 0x21, + 0x29, 0x07, 0x37, 0x93, 0x86, 0x50, 0xe5, 0x19, 0xcc, 0x70, 0x5f, 0x5b, 0x9a, 0xea, 0x86, 0x83, + 0xdf, 0x5b, 0xe2, 0x97, 0x9d, 0x68, 0x2c, 0x9e, 0x35, 0x89, 0xf6, 0x31, 0xd8, 0xd4, 0x40, 0xce, + 0xaa, 0x6e, 0xe9, 0xb0, 0x9d, 0xf4, 0x0f, 0x60, 0xe2, 0x40, 0xd5, 0x2b, 0xfe, 0x1b, 0xa6, 0x7f, + 0x23, 0xe4, 0x50, 0x8b, 0x28, 0x01, 0x0f, 0x6a, 0x4e, 0xf2, 0x4e, 0xf3, 0x2d, 0x19, 0x17, 0xe1, + 0x81, 0xbd, 0x39, 0x7c, 0xe5, 0xfa, 0xa6, 0xec, 0x6d, 0x92, 0xde, 0xdd, 0xde, 0x28, 0x41, 0x1f, + 0xf6, 0x26, 0x20, 0x9e, 0x37, 0xb4, 0xc7, 0xaa, 0x7d, 0x4c, 0x6d, 0xe5, 0x13, 0x09, 0xe6, 0xa2, + 0x37, 0xe3, 0x31, 0x75, 0x1c, 0xb5, 0x4c, 0xc9, 0xff, 0xf4, 0x67, 0xff, 0x83, 0x81, 0xf0, 0x83, + 0x42, 0x8c, 0x1a, 0x9a, 0x57, 0x54, 0x26, 0x91, 0x2c, 0x90, 0xc7, 0xef, 0x17, 0x15, 0x7b, 0xcc, + 0x07, 0x03, 0xbb, 0x0c, 0x3f, 0x3b, 0x0a, 0xc3, 0xf4, 0x84, 0x1a, 0xee, 0x4a, 0x0a, 0x12, 0xc2, + 0xb7, 0x77, 0x92, 0x80, 0x51, 0x6f, 0x99, 0x1c, 0x58, 0xb9, 0x0e, 0x09, 0xe1, 0x23, 0x2d, 0x19, + 0x87, 0xb1, 0x6d, 0x53, 0xa3, 0x3b, 0xa6, 0xed, 0x26, 0x07, 0xd8, 0xea, 0x01, 0x55, 0xb5, 0x0a, + 0x43, 0x95, 0x56, 0x3e, 0x95, 0x60, 0xcc, 0xff, 0x84, 0x43, 0x00, 0x46, 0xde, 0xd9, 0xcf, 0xef, + 0xe7, 0x37, 0x92, 0x03, 0x8c, 0xe1, 0x4e, 0x7e, 0x7b, 0x63, 0x73, 0xfb, 0x7e, 0x52, 0x62, 0x8b, + 0xdd, 0xfd, 0xed, 0x6d, 0xb6, 0x18, 0x24, 0x13, 0x10, 0xdf, 0xdb, 0xcf, 0xe5, 0xf2, 0xf9, 0x8d, + 0xfc, 0x46, 0x32, 0xc6, 0x88, 0xee, 0xad, 0x6f, 0x3e, 0xca, 0x6f, 0x24, 0x87, 0x18, 0xde, 0xfe, + 0xf6, 0xc3, 0xed, 0x27, 0xef, 0x6d, 0x27, 0x87, 0x39, 0x5e, 0xf6, 0xf1, 0xe6, 0xd3, 0xa7, 0xf9, + 0x8d, 0xe4, 0x08, 0xc3, 0x7b, 0x94, 0x5f, 0xdf, 0xcb, 0x6f, 0x24, 0x47, 0xd9, 0xd6, 0xce, 0x6e, + 0x3e, 0xff, 0x78, 0x87, 0x6d, 0x8d, 0xb1, 0x65, 0x6e, 0x7d, 0x3b, 0x97, 0x7f, 0xc4, 0xb8, 0xc4, + 0x99, 0x86, 0xbb, 0xf9, 0xad, 0x7c, 0x8e, 0x6d, 0xc2, 0xda, 0xef, 0x87, 0x60, 0x1c, 0x1d, 0xea, + 0x3f, 0x0e, 0xbe, 0x09, 0x09, 0x7e, 0xaa, 0x7c, 0xbe, 0x16, 0x5c, 0x9e, 0x9a, 0x6f, 0x79, 0xb6, + 0xcd, 0x33, 0xe7, 0x29, 0x03, 0xe4, 0x2e, 0x8c, 0x0b, 0x44, 0x0e, 0x99, 0x0c, 0xa9, 0x58, 0x11, + 0x49, 0xbd, 0x84, 0xeb, 0x4e, 0x81, 0xa6, 0x0c, 0x30, 0xa9, 0xfc, 0xee, 0xf4, 0x29, 0x55, 0x20, + 0xea, 0x2e, 0x35, 0x7a, 0x3b, 0x95, 0x01, 0xf2, 0x2d, 0x48, 0xf0, 0x5c, 0xca, 0xa5, 0x5e, 0x0a, + 0xe9, 0x23, 0x29, 0xf6, 0x0c, 0x15, 0x32, 0x30, 0x76, 0x9f, 0xba, 0x9c, 0x7c, 0x36, 0x24, 0x0f, + 0x33, 0x7b, 0x4a, 0x30, 0x45, 0x19, 0x20, 0x5b, 0x10, 0xf7, 0xf1, 0x1d, 0xc2, 0xf5, 0xeb, 0x54, + 0x13, 0x52, 0xa9, 0x36, 0xdb, 0xde, 0xc5, 0x50, 0x06, 0xde, 0x90, 0x98, 0xf6, 0xbc, 0x90, 0xb5, + 0x68, 0x1f, 0xa9, 0x6f, 0x67, 0x68, 0xbf, 0x01, 0x13, 0x7e, 0x31, 0xe3, 0x3c, 0x16, 0x84, 0x54, + 0x16, 0xad, 0x72, 0x9d, 0xb9, 0xac, 0xfd, 0x24, 0x0e, 0x23, 0xfc, 0x55, 0x87, 0xbc, 0x0b, 0xc0, + 0x7f, 0x61, 0xfe, 0x9f, 0x6b, 0xfb, 0xa1, 0x3f, 0x35, 0xdf, 0xfe, 0x29, 0x48, 0x59, 0xf8, 0xd1, + 0x9f, 0xfe, 0xf6, 0xb3, 0xc1, 0x19, 0x65, 0x72, 0xf5, 0xe4, 0xe6, 0xea, 0x91, 0x59, 0xf4, 0xfe, + 0x97, 0x78, 0x47, 0x5a, 0x21, 0xef, 0x01, 0xf0, 0x56, 0x22, 0xca, 0x37, 0xf2, 0x85, 0x38, 0xc5, + 0x1d, 0xd0, 0xda, 0x72, 0xb4, 0x32, 0xe6, 0xfd, 0x04, 0x63, 0xfc, 0x5d, 0x18, 0x0f, 0x18, 0xef, + 0x51, 0x97, 0xc8, 0xc2, 0x47, 0xdf, 0x28, 0xf7, 0x4e, 0xf6, 0x5f, 0x41, 0xe6, 0xf3, 0xca, 0xb4, + 0xc7, 0xdc, 0xa1, 0xae, 0xc0, 0xdf, 0x80, 0xa4, 0xf8, 0x00, 0x89, 0xea, 0x5f, 0x6e, 0xff, 0x34, + 0xc9, 0xc5, 0x5c, 0x39, 0xeb, 0xdd, 0x52, 0x49, 0xa3, 0xb0, 0x05, 0x65, 0xd6, 0xb7, 0x44, 0x78, + 0x83, 0xa4, 0x4c, 0xde, 0x07, 0x90, 0xf0, 0xfe, 0x0c, 0x81, 0xa2, 0x02, 0x57, 0x47, 0xff, 0x21, + 0xd1, 0xd1, 0x98, 0x14, 0xf2, 0x9f, 0x55, 0xa6, 0x7c, 0xfe, 0x16, 0xa7, 0x63, 0xac, 0xef, 0xf7, + 0x9f, 0x18, 0x66, 0x91, 0xdd, 0xa4, 0x12, 0x67, 0xec, 0x30, 0x31, 0x33, 0x46, 0xa5, 0xe7, 0x4b, + 0x16, 0xaf, 0x20, 0xd3, 0x45, 0x65, 0x81, 0x31, 0x2d, 0x32, 0x2c, 0xaa, 0xad, 0xf2, 0xaf, 0x43, + 0x5e, 0x9d, 0x62, 0x42, 0xb6, 0xfb, 0x4f, 0x28, 0x97, 0x91, 0xf1, 0x5c, 0x2a, 0x19, 0x68, 0xbb, + 0xfa, 0x43, 0xd6, 0x02, 0x7d, 0xe4, 0x29, 0xfd, 0x3c, 0xb9, 0xc6, 0x53, 0x3a, 0x15, 0x51, 0xba, + 0x86, 0x38, 0x82, 0xd2, 0xef, 0x3f, 0x67, 0x3e, 0x92, 0x51, 0x0a, 0x59, 0x69, 0xb1, 0x80, 0xdc, + 0xeb, 0x2b, 0x4f, 0x79, 0x7c, 0x48, 0x2b, 0x1f, 0xed, 0x05, 0xe5, 0x2f, 0x2f, 0xd0, 0x08, 0x11, + 0xfd, 0xc1, 0x1d, 0xf1, 0x86, 0x44, 0xee, 0xc0, 0xc8, 0x03, 0xfc, 0x3b, 0x2f, 0xe9, 0x60, 0x69, + 0x8a, 0xdf, 0x53, 0x8e, 0x94, 0x3b, 0xa4, 0xa5, 0xe3, 0xa0, 0x07, 0x79, 0xff, 0x0f, 0x5f, 0x2c, + 0x4a, 0x9f, 0x7f, 0xb1, 0x28, 0xfd, 0xf5, 0x8b, 0x45, 0xe9, 0x93, 0x2f, 0x17, 0x07, 0x3e, 0xff, + 0x72, 0x71, 0xe0, 0xcf, 0x5f, 0x2e, 0x0e, 0x7c, 0xfb, 0xb5, 0xb2, 0xee, 0x1e, 0xd6, 0x8a, 0x99, + 0x92, 0x59, 0x5d, 0x55, 0xed, 0xaa, 0xaa, 0xa9, 0x96, 0x6d, 0x1e, 0xd1, 0x92, 0xeb, 0xad, 0x56, + 0xbd, 0xbf, 0x12, 0x7f, 0x36, 0x38, 0xbb, 0x8e, 0x80, 0x1d, 0xbe, 0x9d, 0xd9, 0x34, 0x33, 0xeb, + 0x96, 0x5e, 0x1c, 0x41, 0x1d, 0xde, 0xfc, 0x77, 0x00, 0x00, 0x00, 0xff, 0xff, 0xab, 0x9e, 0x06, + 0xd2, 0x38, 0x2d, 0x00, 0x00, } // Reference imports to suppress errors if they are not otherwise used. @@ -3720,6 +3728,13 @@ func (m *JobPreemptRequest) MarshalToSizedBuffer(dAtA []byte) (int, error) { _ = i var l int _ = l + if len(m.Reason) > 0 { + i -= len(m.Reason) + copy(dAtA[i:], m.Reason) + i = encodeVarintSubmit(dAtA, i, uint64(len(m.Reason))) + i-- + dAtA[i] = 0x22 + } if len(m.JobIds) > 0 { for iNdEx := len(m.JobIds) - 1; iNdEx >= 0; iNdEx-- { i -= len(m.JobIds[iNdEx]) @@ -5380,6 +5395,10 @@ func (m *JobPreemptRequest) Size() (n int) { n += 1 + l + sovSubmit(uint64(l)) } } + l = len(m.Reason) + if l > 0 { + n += 1 + l + sovSubmit(uint64(l)) + } return n } @@ -7473,6 +7492,38 @@ func (m *JobPreemptRequest) Unmarshal(dAtA []byte) error { } m.JobIds = append(m.JobIds, string(dAtA[iNdEx:postIndex])) iNdEx = postIndex + case 4: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Reason", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowSubmit + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= uint64(b&0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthSubmit + } + postIndex := iNdEx + intStringLen + if postIndex < 0 { + return ErrInvalidLengthSubmit + } + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.Reason = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipSubmit(dAtA[iNdEx:]) diff --git a/pkg/api/submit.proto b/pkg/api/submit.proto index 03372a719d6..082d6de6ea3 100644 --- a/pkg/api/submit.proto +++ b/pkg/api/submit.proto @@ -64,6 +64,7 @@ message JobPreemptRequest { string queue = 1; string job_set_id = 2; repeated string job_ids = 3; + string reason = 4; } // swagger:model diff --git a/pkg/armadaevents/events.pb.go b/pkg/armadaevents/events.pb.go index a562a1bdd16..9d065f22358 100644 --- a/pkg/armadaevents/events.pb.go +++ b/pkg/armadaevents/events.pb.go @@ -3124,6 +3124,7 @@ func (m *MaxRunsExceeded) GetMessage() string { } type JobRunPreemptedError struct { + Reason string `protobuf:"bytes,1,opt,name=reason,proto3" json:"reason,omitempty"` } func (m *JobRunPreemptedError) Reset() { *m = JobRunPreemptedError{} } @@ -3159,6 +3160,13 @@ func (m *JobRunPreemptedError) XXX_DiscardUnknown() { var xxx_messageInfo_JobRunPreemptedError proto.InternalMessageInfo +func (m *JobRunPreemptedError) GetReason() string { + if m != nil { + return m.Reason + } + return "" +} + type GangJobUnschedulable struct { Message string `protobuf:"bytes,1,opt,name=message,proto3" json:"message,omitempty"` } @@ -3251,6 +3259,7 @@ func (m *JobRejected) GetMessage() string { type JobRunPreempted struct { PreemptedJobId string `protobuf:"bytes,5,opt,name=preempted_job_id,json=preemptedJobId,proto3" json:"preemptedJobId,omitempty"` PreemptedRunId string `protobuf:"bytes,6,opt,name=preempted_run_id,json=preemptedRunId,proto3" json:"preemptedRunId,omitempty"` + Reason string `protobuf:"bytes,7,opt,name=reason,proto3" json:"reason,omitempty"` } func (m *JobRunPreempted) Reset() { *m = JobRunPreempted{} } @@ -3300,6 +3309,13 @@ func (m *JobRunPreempted) GetPreemptedRunId() string { return "" } +func (m *JobRunPreempted) GetReason() string { + if m != nil { + return m.Reason + } + return "" +} + // Message used internally by Armada to see if messages can be propagated through a pulsar partition type PartitionMarker struct { // Group id identifies the group of partition markers, one per partition @@ -3410,7 +3426,8 @@ func (m *JobRunPreemptionRequested) GetRunId() string { // Indicates that a user has requested for the job to be pre-empted. type JobPreemptionRequested struct { - JobId string `protobuf:"bytes,2,opt,name=job_id,json=jobId,proto3" json:"jobId,omitempty"` + JobId string `protobuf:"bytes,2,opt,name=job_id,json=jobId,proto3" json:"jobId,omitempty"` + Reason string `protobuf:"bytes,3,opt,name=reason,proto3" json:"reason,omitempty"` } func (m *JobPreemptionRequested) Reset() { *m = JobPreemptionRequested{} } @@ -3453,6 +3470,13 @@ func (m *JobPreemptionRequested) GetJobId() string { return "" } +func (m *JobPreemptionRequested) GetReason() string { + if m != nil { + return m.Reason + } + return "" +} + // Indicates that the scheduler is happy with the job type JobValidated struct { Pools []string `protobuf:"bytes,2,rep,name=pools,proto3" json:"pools,omitempty"` @@ -3620,231 +3644,233 @@ func init() { func init() { proto.RegisterFile("pkg/armadaevents/events.proto", fileDescriptor_6aab92ca59e015f8) } var fileDescriptor_6aab92ca59e015f8 = []byte{ - // 3580 bytes of a gzipped FileDescriptorProto - 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xe4, 0x1b, 0x4d, 0x6f, 0x1b, 0xc7, - 0xd5, 0xcb, 0x6f, 0x3e, 0x4a, 0x14, 0x3d, 0xfa, 0x30, 0xad, 0xc4, 0xa2, 0x4c, 0xa7, 0x8d, 0x1d, - 0x24, 0x94, 0xe3, 0x34, 0x45, 0x3e, 0x8a, 0x04, 0xa2, 0xad, 0xd8, 0x56, 0x2c, 0x5b, 0xa1, 0xec, - 0xd4, 0x2d, 0x02, 0x30, 0x4b, 0xee, 0x88, 0x5a, 0x8b, 0xdc, 0xdd, 0xec, 0x87, 0x22, 0x01, 0x01, - 0x9a, 0x14, 0x69, 0xcf, 0xb9, 0x14, 0x28, 0x7a, 0x69, 0x2e, 0x45, 0x91, 0xa2, 0x3d, 0xb6, 0xd7, - 0x5e, 0x7b, 0x28, 0x8a, 0x1c, 0x8b, 0xa2, 0x21, 0x8a, 0x04, 0xbd, 0xf0, 0xd0, 0xdf, 0x50, 0xcc, - 0xc7, 0xee, 0xce, 0x2c, 0x87, 0xb6, 0xe4, 0x58, 0x46, 0x9a, 0x9e, 0xa4, 0x7d, 0xdf, 0x3b, 0xef, - 0xcd, 0xdb, 0xf7, 0xde, 0x0c, 0xe1, 0x8c, 0xb3, 0xdb, 0x5b, 0xd1, 0xdd, 0x81, 0x6e, 0xe8, 0x78, - 0x0f, 0x5b, 0xbe, 0xb7, 0xc2, 0xfe, 0x34, 0x1c, 0xd7, 0xf6, 0x6d, 0x34, 0x25, 0xa2, 0x16, 0xeb, - 0xbb, 0x2f, 0x79, 0x0d, 0xd3, 0x5e, 0xd1, 0x1d, 0x73, 0xa5, 0x6b, 0xbb, 0x78, 0x65, 0xef, 0xf9, - 0x95, 0x1e, 0xb6, 0xb0, 0xab, 0xfb, 0xd8, 0x60, 0x1c, 0x8b, 0xe7, 0x05, 0x1a, 0x0b, 0xfb, 0xef, - 0xdb, 0xee, 0xae, 0x69, 0xf5, 0x54, 0x94, 0xb5, 0x9e, 0x6d, 0xf7, 0xfa, 0x78, 0x85, 0x3e, 0x75, - 0x82, 0xed, 0x15, 0xdf, 0x1c, 0x60, 0xcf, 0xd7, 0x07, 0x0e, 0x27, 0xf8, 0x5e, 0x2c, 0x6a, 0xa0, - 0x77, 0x77, 0x4c, 0x0b, 0xbb, 0x07, 0x2b, 0xd4, 0x5e, 0xc7, 0x5c, 0x71, 0xb1, 0x67, 0x07, 0x6e, - 0x17, 0x8f, 0x89, 0x7d, 0xc5, 0xb4, 0x7c, 0xec, 0x5a, 0x7a, 0x7f, 0xc5, 0xeb, 0xee, 0x60, 0x23, - 0xe8, 0x63, 0x37, 0xfe, 0xcf, 0xee, 0xdc, 0xc3, 0x5d, 0xdf, 0x1b, 0x03, 0x30, 0xde, 0xfa, 0x17, - 0xf3, 0x30, 0xbd, 0x46, 0xde, 0x75, 0x0b, 0xbf, 0x17, 0x60, 0xab, 0x8b, 0xd1, 0x05, 0xc8, 0xbe, - 0x17, 0xe0, 0x00, 0x57, 0xb5, 0x65, 0xed, 0x7c, 0xb1, 0x39, 0x3b, 0x1a, 0xd6, 0x66, 0x28, 0xe0, - 0x59, 0x7b, 0x60, 0xfa, 0x78, 0xe0, 0xf8, 0x07, 0x2d, 0x46, 0x81, 0x5e, 0x81, 0xa9, 0x7b, 0x76, - 0xa7, 0xed, 0x61, 0xbf, 0x6d, 0xe9, 0x03, 0x5c, 0x4d, 0x51, 0x8e, 0xea, 0x68, 0x58, 0x9b, 0xbb, - 0x67, 0x77, 0xb6, 0xb0, 0x7f, 0x53, 0x1f, 0x88, 0x6c, 0x10, 0x43, 0xd1, 0x73, 0x90, 0x0f, 0x3c, - 0xec, 0xb6, 0x4d, 0xa3, 0x9a, 0xa6, 0x6c, 0x73, 0xa3, 0x61, 0xad, 0x42, 0x40, 0xd7, 0x0d, 0x81, - 0x25, 0xc7, 0x20, 0xe8, 0x59, 0xc8, 0xf5, 0x5c, 0x3b, 0x70, 0xbc, 0x6a, 0x66, 0x39, 0x1d, 0x52, - 0x33, 0x88, 0x48, 0xcd, 0x20, 0xe8, 0x16, 0xe4, 0x98, 0x03, 0xab, 0xd9, 0xe5, 0xf4, 0xf9, 0xd2, - 0xa5, 0xb3, 0x0d, 0xd1, 0xab, 0x0d, 0xe9, 0x85, 0xd9, 0x13, 0x13, 0xc8, 0xf0, 0xa2, 0x40, 0x1e, - 0x07, 0x7f, 0x9a, 0x85, 0x2c, 0xa5, 0x43, 0x6f, 0x42, 0xbe, 0xeb, 0x62, 0xb2, 0xfa, 0x55, 0xb4, - 0xac, 0x9d, 0x2f, 0x5d, 0x5a, 0x6c, 0x30, 0xaf, 0x36, 0x42, 0xaf, 0x36, 0x6e, 0x87, 0x5e, 0x6d, - 0xce, 0x8f, 0x86, 0xb5, 0x93, 0x9c, 0x5c, 0x90, 0x1a, 0x4a, 0x40, 0x9b, 0x50, 0xf4, 0x82, 0xce, - 0xc0, 0xf4, 0xd7, 0xed, 0x0e, 0x5d, 0xef, 0xd2, 0xa5, 0x53, 0xb2, 0xa9, 0x5b, 0x21, 0xba, 0x79, - 0x6a, 0x34, 0xac, 0xcd, 0x46, 0xd4, 0xb1, 0xb4, 0x6b, 0x27, 0x5a, 0xb1, 0x10, 0xb4, 0x03, 0x33, - 0x2e, 0x76, 0x5c, 0xd3, 0x76, 0x4d, 0xdf, 0xf4, 0x30, 0x91, 0x9b, 0xa2, 0x72, 0xcf, 0xc8, 0x72, - 0x5b, 0x32, 0x51, 0xf3, 0xcc, 0x68, 0x58, 0x3b, 0x9d, 0xe0, 0x94, 0x74, 0x24, 0xc5, 0x22, 0x1f, - 0x50, 0x02, 0xb4, 0x85, 0x7d, 0xea, 0xcb, 0xd2, 0xa5, 0xe5, 0xfb, 0x2a, 0xdb, 0xc2, 0x7e, 0x73, - 0x79, 0x34, 0xac, 0x3d, 0x39, 0xce, 0x2f, 0xa9, 0x54, 0xc8, 0x47, 0x7d, 0xa8, 0x88, 0x50, 0x83, - 0xbc, 0x60, 0x86, 0xea, 0x5c, 0x9a, 0xac, 0x93, 0x50, 0x35, 0x97, 0x46, 0xc3, 0xda, 0x62, 0x92, - 0x57, 0xd2, 0x37, 0x26, 0x99, 0xf8, 0xa7, 0xab, 0x5b, 0x5d, 0xdc, 0x27, 0x6a, 0xb2, 0x2a, 0xff, - 0x5c, 0x0e, 0xd1, 0xcc, 0x3f, 0x11, 0xb5, 0xec, 0x9f, 0x08, 0x8c, 0xde, 0x81, 0xa9, 0xe8, 0x81, - 0xac, 0x57, 0x8e, 0xc7, 0x90, 0x5a, 0x28, 0x59, 0xa9, 0xc5, 0xd1, 0xb0, 0xb6, 0x20, 0xf2, 0x48, - 0xa2, 0x25, 0x69, 0xb1, 0xf4, 0x3e, 0x5b, 0x99, 0xfc, 0x64, 0xe9, 0x8c, 0x42, 0x94, 0xde, 0x1f, - 0x5f, 0x11, 0x49, 0x1a, 0x91, 0x4e, 0x36, 0x70, 0xd0, 0xed, 0x62, 0x6c, 0x60, 0xa3, 0x5a, 0x50, - 0x49, 0x5f, 0x17, 0x28, 0x98, 0x74, 0x91, 0x47, 0x96, 0x2e, 0x62, 0xc8, 0x5a, 0xdf, 0xb3, 0x3b, - 0x6b, 0xae, 0x6b, 0xbb, 0x5e, 0xb5, 0xa8, 0x5a, 0xeb, 0xf5, 0x10, 0xcd, 0xd6, 0x3a, 0xa2, 0x96, - 0xd7, 0x3a, 0x02, 0x73, 0x7b, 0x5b, 0x81, 0x75, 0x03, 0xeb, 0x1e, 0x36, 0xaa, 0x30, 0xc1, 0xde, - 0x88, 0x22, 0xb2, 0x37, 0x82, 0x8c, 0xd9, 0x1b, 0x61, 0x90, 0x01, 0x65, 0xf6, 0xbc, 0xea, 0x79, - 0x66, 0xcf, 0xc2, 0x46, 0xb5, 0x44, 0xe5, 0x3f, 0xa9, 0x92, 0x1f, 0xd2, 0x34, 0x9f, 0x1c, 0x0d, - 0x6b, 0x55, 0x99, 0x4f, 0xd2, 0x91, 0x90, 0x89, 0xde, 0x85, 0x69, 0x06, 0x69, 0x05, 0x96, 0x65, - 0x5a, 0xbd, 0xea, 0x14, 0x55, 0xf2, 0x84, 0x4a, 0x09, 0x27, 0x69, 0x3e, 0x31, 0x1a, 0xd6, 0x4e, - 0x49, 0x5c, 0x92, 0x0a, 0x59, 0x20, 0xc9, 0x18, 0x0c, 0x10, 0x3b, 0x76, 0x5a, 0x95, 0x31, 0xd6, - 0x65, 0x22, 0x96, 0x31, 0x12, 0x9c, 0x72, 0xc6, 0x48, 0x20, 0x63, 0x7f, 0x70, 0x27, 0x97, 0x27, - 0xfb, 0x83, 0xfb, 0x59, 0xf0, 0x87, 0xc2, 0xd5, 0x92, 0x34, 0xf4, 0xa1, 0x06, 0xf3, 0x9e, 0xaf, - 0x5b, 0x86, 0xde, 0xb7, 0x2d, 0x7c, 0xdd, 0xea, 0xb9, 0xd8, 0xf3, 0xae, 0x5b, 0xdb, 0x76, 0xb5, - 0x42, 0xf5, 0x9c, 0x4b, 0x24, 0x56, 0x15, 0x69, 0xf3, 0xdc, 0x68, 0x58, 0xab, 0x29, 0xa5, 0x48, - 0x9a, 0xd5, 0x8a, 0xd0, 0x3e, 0xcc, 0x86, 0x1f, 0xe9, 0x3b, 0xbe, 0xd9, 0x37, 0x3d, 0xdd, 0x37, - 0x6d, 0xab, 0x7a, 0x92, 0xea, 0x3f, 0x9b, 0xcc, 0x4f, 0x63, 0x84, 0xcd, 0xb3, 0xa3, 0x61, 0xed, - 0x8c, 0x42, 0x82, 0xa4, 0x5b, 0xa5, 0x22, 0x76, 0xe2, 0xa6, 0x8b, 0x09, 0x21, 0x36, 0xaa, 0xb3, - 0x93, 0x9d, 0x18, 0x11, 0x89, 0x4e, 0x8c, 0x80, 0x2a, 0x27, 0x46, 0x48, 0xa2, 0xc9, 0xd1, 0x5d, - 0xdf, 0x24, 0x6a, 0x37, 0x74, 0x77, 0x17, 0xbb, 0xd5, 0x39, 0x95, 0xa6, 0x4d, 0x99, 0x88, 0x69, - 0x4a, 0x70, 0xca, 0x9a, 0x12, 0x48, 0xf4, 0x89, 0x06, 0xb2, 0x69, 0xa6, 0x6d, 0xb5, 0xc8, 0x47, - 0xdb, 0x23, 0xaf, 0x37, 0x4f, 0x95, 0x3e, 0x7d, 0x9f, 0xd7, 0x13, 0xc9, 0x9b, 0x4f, 0x8f, 0x86, - 0xb5, 0x73, 0x13, 0xa5, 0x49, 0x86, 0x4c, 0x56, 0x8a, 0xee, 0x42, 0x89, 0x20, 0x31, 0x2d, 0x7f, - 0x8c, 0xea, 0x02, 0xb5, 0xe1, 0xf4, 0xb8, 0x0d, 0x9c, 0xa0, 0x79, 0x7a, 0x34, 0xac, 0xcd, 0x0b, - 0x1c, 0x92, 0x1e, 0x51, 0x14, 0xfa, 0x58, 0x03, 0x12, 0xe8, 0xaa, 0x37, 0x3d, 0x45, 0xb5, 0x3c, - 0x35, 0xa6, 0x45, 0xf5, 0x9a, 0x4f, 0x8d, 0x86, 0xb5, 0x65, 0xb5, 0x1c, 0x49, 0xf7, 0x04, 0x5d, - 0x71, 0x1c, 0x45, 0x1f, 0x89, 0x6a, 0x75, 0x72, 0x1c, 0x45, 0x44, 0x62, 0x1c, 0x45, 0x40, 0x55, - 0x1c, 0x45, 0x48, 0x9e, 0x0c, 0xde, 0xd6, 0xfb, 0xa6, 0x41, 0x8b, 0xa9, 0xd3, 0x13, 0x92, 0x41, - 0x44, 0x11, 0x25, 0x83, 0x08, 0x32, 0x96, 0x0c, 0x62, 0xda, 0x3c, 0x64, 0xa9, 0x88, 0xfa, 0x3f, - 0x72, 0x30, 0xab, 0xd8, 0x6a, 0x08, 0xc3, 0x74, 0xb8, 0x8f, 0xda, 0x26, 0x49, 0x12, 0x69, 0xd5, - 0x2a, 0xbf, 0x19, 0x74, 0xb0, 0x6b, 0x61, 0x1f, 0x7b, 0xa1, 0x0c, 0x9a, 0x25, 0xa8, 0x25, 0xae, - 0x00, 0x11, 0x6a, 0xbb, 0x29, 0x11, 0x8e, 0x7e, 0xa5, 0x41, 0x75, 0xa0, 0xef, 0xb7, 0x43, 0xa0, - 0xd7, 0xde, 0xb6, 0xdd, 0xb6, 0x83, 0x5d, 0xd3, 0x36, 0x68, 0x25, 0x5b, 0xba, 0xf4, 0x83, 0x07, - 0xe6, 0x85, 0xc6, 0x86, 0xbe, 0x1f, 0x82, 0xbd, 0x37, 0x6c, 0x77, 0x93, 0xb2, 0xaf, 0x59, 0xbe, - 0x7b, 0xc0, 0x12, 0xd6, 0x40, 0x85, 0x17, 0x6c, 0x9a, 0x57, 0x12, 0xa0, 0x5f, 0x68, 0xb0, 0xe0, - 0xdb, 0xbe, 0xde, 0x6f, 0x77, 0x83, 0x41, 0xd0, 0xd7, 0x7d, 0x73, 0x0f, 0xb7, 0x03, 0x4f, 0xef, - 0x61, 0x5e, 0x36, 0xbf, 0xfa, 0x60, 0xd3, 0x6e, 0x13, 0xfe, 0xcb, 0x11, 0xfb, 0x1d, 0xc2, 0xcd, - 0x2c, 0xab, 0x8f, 0x86, 0xb5, 0x25, 0x5f, 0x81, 0x16, 0x0c, 0x9b, 0x53, 0xe1, 0xd1, 0x33, 0x90, - 0x23, 0x6d, 0x85, 0x69, 0xd0, 0xea, 0x88, 0xb7, 0x20, 0xf7, 0xec, 0x8e, 0xd4, 0x18, 0x64, 0x29, - 0x80, 0xd0, 0xba, 0x81, 0x45, 0x68, 0xf3, 0x31, 0xad, 0x1b, 0x58, 0x32, 0x2d, 0x05, 0x2c, 0x7e, - 0xaa, 0xc1, 0xe2, 0xe4, 0xa5, 0x44, 0xe7, 0x20, 0xbd, 0x8b, 0x0f, 0x78, 0xdb, 0x73, 0x72, 0x34, - 0xac, 0x4d, 0xef, 0xe2, 0x03, 0x41, 0x0a, 0xc1, 0xa2, 0x1f, 0x41, 0x76, 0x4f, 0xef, 0x07, 0x98, - 0x57, 0xd5, 0x8d, 0x06, 0xeb, 0xd8, 0x1a, 0x62, 0xc7, 0xd6, 0x70, 0x76, 0x7b, 0x04, 0xd0, 0x08, - 0xbd, 0xde, 0x78, 0x2b, 0xd0, 0x2d, 0xdf, 0xf4, 0x0f, 0x98, 0x79, 0x54, 0x80, 0x68, 0x1e, 0x05, - 0xbc, 0x92, 0x7a, 0x49, 0x5b, 0xfc, 0xb5, 0x06, 0xa7, 0x27, 0x2e, 0xe9, 0x37, 0xc1, 0xc2, 0xf5, - 0x4c, 0x41, 0xab, 0xa4, 0xd6, 0x33, 0x85, 0x54, 0x25, 0x5d, 0xff, 0x7d, 0x0e, 0x8a, 0x51, 0x83, - 0x82, 0xae, 0x41, 0xc5, 0xc0, 0x46, 0xe0, 0xf4, 0xcd, 0x2e, 0x8d, 0x0d, 0xe2, 0x14, 0xd6, 0x11, - 0xd2, 0xec, 0x20, 0xe1, 0x24, 0xf7, 0xcc, 0x24, 0x50, 0xe8, 0x12, 0x14, 0x78, 0x21, 0x7e, 0x40, - 0xf7, 0xe5, 0x74, 0x73, 0x61, 0x34, 0xac, 0xa1, 0x10, 0x26, 0xb0, 0x46, 0x74, 0xa8, 0x05, 0xc0, - 0x3a, 0xdb, 0x0d, 0xec, 0xeb, 0xbc, 0x25, 0xa8, 0xca, 0xf1, 0x7b, 0x2b, 0xc2, 0xb3, 0x1e, 0x35, - 0xa6, 0x17, 0x7b, 0xd4, 0x18, 0x8a, 0xde, 0x01, 0x18, 0xe8, 0xa6, 0xc5, 0xf8, 0x78, 0xfd, 0x5f, - 0x9f, 0x94, 0x21, 0x36, 0x22, 0x4a, 0x26, 0x3d, 0xe6, 0x14, 0xa5, 0xc7, 0x50, 0x74, 0x0b, 0xf2, - 0xbc, 0x17, 0xaf, 0xe6, 0xe8, 0x76, 0x5b, 0x9a, 0x24, 0x9a, 0x8b, 0xa5, 0xdd, 0x24, 0x67, 0x11, - 0xbb, 0x49, 0x0e, 0x22, 0xcb, 0xd6, 0x37, 0xb7, 0xb1, 0x6f, 0x0e, 0x30, 0xdd, 0x0d, 0x7c, 0xd9, - 0x42, 0x98, 0xb8, 0x6c, 0x21, 0x0c, 0xbd, 0x04, 0xa0, 0xfb, 0x1b, 0xb6, 0xe7, 0xdf, 0xb2, 0xba, - 0x98, 0x56, 0xf4, 0x05, 0x66, 0x7e, 0x0c, 0x15, 0xcd, 0x8f, 0xa1, 0xe8, 0x55, 0x28, 0x39, 0xfc, - 0x0b, 0xd2, 0xe9, 0x63, 0x5a, 0xb1, 0x17, 0xd8, 0x07, 0x4f, 0x00, 0x0b, 0xbc, 0x22, 0x35, 0xba, - 0x0a, 0x33, 0x5d, 0xdb, 0xea, 0x06, 0xae, 0x8b, 0xad, 0xee, 0xc1, 0x96, 0xbe, 0x8d, 0x69, 0x75, - 0x5e, 0x60, 0xa1, 0x92, 0x40, 0x89, 0xa1, 0x92, 0x40, 0xa1, 0x17, 0xa1, 0x18, 0x4d, 0x36, 0x68, - 0x01, 0x5e, 0xe4, 0x8d, 0x72, 0x08, 0x14, 0x98, 0x63, 0x4a, 0x62, 0xbc, 0xe9, 0x5d, 0xe1, 0x41, - 0x87, 0x69, 0x51, 0xcd, 0x8d, 0x17, 0xc0, 0xa2, 0xf1, 0x02, 0x58, 0xc8, 0x4f, 0xe5, 0x07, 0xe5, - 0xa7, 0x68, 0xbb, 0x4c, 0x57, 0xca, 0xf5, 0xbf, 0x6a, 0x30, 0xa7, 0x8a, 0x97, 0x44, 0xec, 0x6a, - 0x8f, 0x24, 0x76, 0xdf, 0x86, 0x82, 0x63, 0x1b, 0x6d, 0xcf, 0xc1, 0x5d, 0x9e, 0x09, 0x12, 0x91, - 0xbb, 0x69, 0x1b, 0x5b, 0x0e, 0xee, 0xfe, 0xd0, 0xf4, 0x77, 0x56, 0xf7, 0x6c, 0xd3, 0xb8, 0x61, - 0x7a, 0x3c, 0xc4, 0x1c, 0x86, 0x91, 0x3e, 0xaf, 0x79, 0x0e, 0x6c, 0x16, 0x20, 0xc7, 0xb4, 0xd4, - 0xff, 0x96, 0x86, 0x4a, 0x32, 0x46, 0xff, 0x97, 0x5e, 0x05, 0xdd, 0x85, 0xbc, 0xc9, 0xaa, 0x77, - 0xfe, 0xf5, 0xff, 0x8e, 0x90, 0x2b, 0x1b, 0xf1, 0x28, 0xaf, 0xb1, 0xf7, 0x7c, 0x83, 0x97, 0xf9, - 0x74, 0x09, 0xa8, 0x64, 0xce, 0x29, 0x4b, 0xe6, 0x40, 0xd4, 0x82, 0xbc, 0x87, 0xdd, 0x3d, 0xb3, - 0x8b, 0x79, 0x26, 0xaa, 0x89, 0x92, 0xbb, 0xb6, 0x8b, 0x89, 0xcc, 0x2d, 0x46, 0x12, 0xcb, 0xe4, - 0x3c, 0xb2, 0x4c, 0x0e, 0x44, 0x6f, 0x43, 0xb1, 0x6b, 0x5b, 0xdb, 0x66, 0x6f, 0x43, 0x77, 0x78, - 0x2e, 0x3a, 0xa3, 0x92, 0x7a, 0x39, 0x24, 0xe2, 0x13, 0x89, 0xf0, 0x31, 0x31, 0x91, 0x88, 0xa8, - 0x62, 0x87, 0xfe, 0x27, 0x03, 0x10, 0x3b, 0x07, 0xbd, 0x0c, 0x25, 0xbc, 0x8f, 0xbb, 0x81, 0x6f, - 0xd3, 0x29, 0x9d, 0x16, 0x0f, 0xf7, 0x42, 0xb0, 0x14, 0xf0, 0x10, 0x43, 0xc9, 0xae, 0xb4, 0xf4, - 0x01, 0xf6, 0x1c, 0xbd, 0x1b, 0x4e, 0x05, 0xa9, 0x31, 0x11, 0x50, 0xdc, 0x95, 0x11, 0x10, 0x7d, - 0x17, 0x32, 0x74, 0x8e, 0xc8, 0x06, 0x82, 0x68, 0x34, 0xac, 0x95, 0x2d, 0x79, 0x82, 0x48, 0xf1, - 0xe8, 0x75, 0x98, 0xde, 0x8d, 0x02, 0x8f, 0xd8, 0x96, 0xa1, 0x0c, 0xb4, 0x2c, 0x8b, 0x11, 0x92, - 0x75, 0x53, 0x22, 0x1c, 0x6d, 0x43, 0x49, 0xb7, 0x2c, 0xdb, 0xa7, 0x1f, 0x9c, 0x70, 0x48, 0x78, - 0x61, 0x52, 0x98, 0x36, 0x56, 0x63, 0x5a, 0x56, 0xdb, 0xd0, 0x4c, 0x21, 0x48, 0x10, 0x33, 0x85, - 0x00, 0x46, 0x2d, 0xc8, 0xf5, 0xf5, 0x0e, 0xee, 0x87, 0x19, 0xfe, 0xa9, 0x89, 0x2a, 0x6e, 0x50, - 0x32, 0x26, 0x9d, 0x8e, 0x22, 0x19, 0x9f, 0x38, 0x8a, 0x64, 0x90, 0xc5, 0x6d, 0xa8, 0x24, 0xed, - 0x39, 0x5c, 0x61, 0x70, 0x41, 0x2c, 0x0c, 0x8a, 0x0f, 0x2c, 0x45, 0x74, 0x28, 0x09, 0x46, 0x1d, - 0x87, 0x8a, 0xfa, 0x67, 0x1a, 0xcc, 0xa9, 0xf6, 0x2e, 0xda, 0x10, 0x76, 0xbc, 0xc6, 0x07, 0x1e, - 0x8a, 0x50, 0xe7, 0xbc, 0x13, 0xb6, 0x7a, 0xbc, 0xd1, 0x9b, 0x50, 0xb6, 0x6c, 0x03, 0xb7, 0x75, - 0xa2, 0xa0, 0x6f, 0x7a, 0x7e, 0x35, 0x45, 0x87, 0xc8, 0x74, 0x50, 0x42, 0x30, 0xab, 0x21, 0x42, - 0xe0, 0x9e, 0x96, 0x10, 0xf5, 0xf7, 0x61, 0x26, 0x31, 0xc6, 0x94, 0xca, 0x94, 0xd4, 0x21, 0xcb, - 0x94, 0xf8, 0xdb, 0x91, 0x3e, 0xdc, 0xb7, 0xa3, 0xfe, 0xb3, 0x14, 0x94, 0x84, 0x9e, 0x12, 0xdd, - 0x83, 0x19, 0xfe, 0x1d, 0x33, 0xad, 0x1e, 0xeb, 0x5d, 0x52, 0x7c, 0xc0, 0x31, 0x36, 0xe3, 0x5f, - 0xb7, 0x3b, 0x5b, 0x11, 0x2d, 0x6d, 0x5d, 0xe8, 0xfc, 0xc9, 0x93, 0x60, 0x82, 0xe2, 0xb2, 0x8c, - 0x41, 0x77, 0x61, 0x21, 0x70, 0x48, 0x47, 0xd5, 0xf6, 0xf8, 0xb4, 0xbc, 0x6d, 0x05, 0x83, 0x0e, - 0x76, 0xa9, 0xf5, 0x59, 0x56, 0xe3, 0x33, 0x8a, 0x70, 0x9c, 0x7e, 0x93, 0xe2, 0xc5, 0x1a, 0x5f, - 0x85, 0x17, 0xd6, 0x21, 0x73, 0xc8, 0x75, 0xb8, 0x06, 0x68, 0x7c, 0x8e, 0x2c, 0xf9, 0x40, 0x3b, - 0x9c, 0x0f, 0xea, 0xfb, 0x50, 0x49, 0x4e, 0x87, 0x1f, 0x93, 0x2f, 0x77, 0xa1, 0x18, 0xcd, 0x76, - 0xd1, 0xb3, 0x90, 0x73, 0xb1, 0xee, 0xd9, 0x16, 0xdf, 0x2d, 0x74, 0xdb, 0x33, 0x88, 0xb8, 0xed, - 0x19, 0xe4, 0x21, 0x94, 0xdd, 0x86, 0x29, 0xb6, 0x48, 0x6f, 0x98, 0x7d, 0x1f, 0xbb, 0xe8, 0x0a, - 0xe4, 0x3c, 0x5f, 0xf7, 0xb1, 0x57, 0xd5, 0x96, 0xd3, 0xe7, 0xcb, 0x97, 0x16, 0xc6, 0x07, 0xb7, - 0x04, 0xcd, 0xec, 0x60, 0x94, 0xa2, 0x1d, 0x0c, 0x52, 0xff, 0xa9, 0x06, 0x53, 0xe2, 0x7c, 0xfa, - 0xd1, 0x88, 0x3d, 0xda, 0x62, 0xd4, 0xad, 0xd0, 0x06, 0x3e, 0x99, 0x3e, 0xee, 0xa5, 0xfc, 0x4c, - 0x63, 0x6b, 0x19, 0x8d, 0x32, 0x7b, 0xf1, 0xf8, 0x80, 0x6c, 0x14, 0x8f, 0x26, 0x94, 0xc3, 0x8e, - 0x0f, 0x68, 0xda, 0x91, 0xd8, 0xc5, 0xb4, 0x23, 0x21, 0x1e, 0xc2, 0xd6, 0x4f, 0xb3, 0xd4, 0xd6, - 0x78, 0x50, 0x9d, 0xf8, 0x8e, 0xa7, 0x8f, 0xf0, 0x1d, 0x7f, 0x0e, 0xf2, 0x34, 0x71, 0x46, 0xdb, - 0x94, 0x2e, 0x2c, 0x01, 0xc9, 0x87, 0x74, 0x0c, 0x72, 0x9f, 0x74, 0x91, 0xfd, 0x9a, 0xe9, 0xa2, - 0x0d, 0xa7, 0x77, 0x74, 0xaf, 0x1d, 0x26, 0x38, 0xa3, 0xad, 0xfb, 0xed, 0x68, 0xbf, 0xe6, 0x68, - 0xf5, 0x4e, 0x47, 0x5f, 0x3b, 0xba, 0xb7, 0x15, 0xd2, 0xac, 0xfa, 0x9b, 0xe3, 0xbb, 0x77, 0x41, - 0x4d, 0x81, 0xee, 0xc0, 0xbc, 0x5a, 0x78, 0x9e, 0x5a, 0x4e, 0x27, 0xb3, 0xde, 0x7d, 0x25, 0xcf, - 0x2a, 0xd0, 0xe8, 0x23, 0x0d, 0xaa, 0xe4, 0x4b, 0xe6, 0xe2, 0xf7, 0x02, 0xd3, 0xc5, 0x03, 0x12, - 0x16, 0x6d, 0x7b, 0x0f, 0xbb, 0x7d, 0xfd, 0x80, 0x1f, 0x72, 0x9c, 0x1d, 0x4f, 0xdb, 0x9b, 0xb6, - 0xd1, 0x12, 0x18, 0xd8, 0xab, 0x39, 0x32, 0xf0, 0x16, 0x13, 0x22, 0xbe, 0x9a, 0x9a, 0x42, 0x08, - 0x21, 0x38, 0xc2, 0x38, 0xa5, 0xf4, 0xa0, 0x71, 0x0a, 0xa9, 0xd6, 0x1c, 0xdb, 0xee, 0xd3, 0xe6, - 0x89, 0x57, 0x6b, 0xe4, 0x59, 0xac, 0xd6, 0xc8, 0xb3, 0x38, 0x31, 0x58, 0xcf, 0x14, 0x0a, 0x95, - 0x62, 0xfd, 0x0b, 0x0d, 0xca, 0xf2, 0xb9, 0xc8, 0xf8, 0x86, 0x4a, 0x1f, 0xfb, 0x86, 0xca, 0x1c, - 0x61, 0x35, 0xb2, 0x0f, 0x5a, 0x0d, 0x69, 0x2e, 0xf2, 0x4f, 0x0d, 0xa6, 0xa5, 0x23, 0x99, 0x6f, - 0xd7, 0xeb, 0xfd, 0x32, 0x05, 0x0b, 0x6a, 0x53, 0x8f, 0xa5, 0xfd, 0xbb, 0x06, 0xa4, 0x90, 0xbb, - 0x1e, 0x17, 0x3a, 0xf3, 0x63, 0xdd, 0x1f, 0x5d, 0xa6, 0xb0, 0x0a, 0x1c, 0x3b, 0xad, 0x09, 0xd9, - 0xd1, 0x5d, 0x28, 0x99, 0xc2, 0xb9, 0x50, 0x5a, 0x35, 0xbe, 0x17, 0x4f, 0x83, 0xd8, 0x40, 0x60, - 0xc2, 0x19, 0x90, 0x28, 0xaa, 0x99, 0x83, 0x0c, 0xa9, 0xc4, 0xea, 0x7b, 0x90, 0xe7, 0xe6, 0xa0, - 0x17, 0xa0, 0x48, 0x73, 0x27, 0xed, 0x68, 0x58, 0xd9, 0x4c, 0x4b, 0x0a, 0x02, 0x4c, 0xdc, 0x8b, - 0x28, 0x84, 0x30, 0xf4, 0x7d, 0x00, 0x92, 0x2e, 0x78, 0xd6, 0x4c, 0xd1, 0xdc, 0x43, 0x3b, 0x27, - 0xc7, 0x36, 0xc6, 0x52, 0x65, 0x31, 0x02, 0xd6, 0xff, 0x90, 0x82, 0x92, 0x78, 0x12, 0xf5, 0x50, - 0xca, 0x3f, 0x80, 0xb0, 0xab, 0x6d, 0xeb, 0x86, 0x41, 0xfe, 0xe2, 0xf0, 0xc3, 0xb6, 0x32, 0x71, - 0x91, 0xc2, 0xff, 0x57, 0x43, 0x0e, 0xd6, 0xc3, 0xd0, 0xd3, 0x76, 0x33, 0x81, 0x12, 0xb4, 0x56, - 0x92, 0xb8, 0xc5, 0x5d, 0x98, 0x57, 0x8a, 0x12, 0x3b, 0x8f, 0xec, 0xa3, 0xea, 0x3c, 0x7e, 0x93, - 0x85, 0x79, 0xe5, 0x09, 0x60, 0x22, 0x82, 0xd3, 0x8f, 0x24, 0x82, 0x7f, 0xae, 0xa9, 0x56, 0x96, - 0x8d, 0xff, 0x5f, 0x3e, 0xc4, 0xb1, 0xe4, 0xa3, 0x5a, 0x63, 0x39, 0x2c, 0xb2, 0x0f, 0x15, 0x93, - 0xb9, 0xc3, 0xc6, 0x24, 0xba, 0xc8, 0x9a, 0x38, 0xaa, 0x8b, 0x0d, 0xe7, 0xc3, 0x1d, 0x9a, 0x50, - 0x95, 0xe7, 0x20, 0xd2, 0xd7, 0x87, 0x1c, 0x6c, 0x74, 0x50, 0x88, 0xfb, 0x7a, 0x4e, 0x93, 0x9c, - 0x1e, 0x4c, 0x89, 0x70, 0x21, 0xfb, 0x15, 0x8f, 0x90, 0xfd, 0xe0, 0x81, 0x27, 0x07, 0x8f, 0x33, - 0x36, 0xa5, 0x54, 0x3b, 0xd4, 0x60, 0x26, 0x71, 0xf0, 0xfe, 0xed, 0xfa, 0x96, 0x7c, 0xa8, 0x41, - 0x31, 0xba, 0xd7, 0x81, 0x56, 0x21, 0x87, 0xd9, 0xdd, 0x00, 0x96, 0x76, 0x66, 0x13, 0xf7, 0xb6, - 0x08, 0x8e, 0xdf, 0xd4, 0x4a, 0x5c, 0x07, 0x68, 0x71, 0xc6, 0x87, 0x28, 0x98, 0xff, 0xa8, 0x85, - 0x05, 0xf3, 0x98, 0x15, 0xe9, 0xaf, 0x6f, 0xc5, 0xf1, 0x2d, 0xdd, 0x9f, 0x8b, 0x90, 0xa5, 0xb6, - 0x90, 0xe6, 0xd5, 0xc7, 0xee, 0xc0, 0xb4, 0xf4, 0x3e, 0x0d, 0xc5, 0x02, 0xdb, 0xd5, 0x21, 0x4c, - 0xdc, 0xd5, 0x21, 0x0c, 0xed, 0xc0, 0x4c, 0x3c, 0x12, 0xa3, 0x62, 0xd4, 0x17, 0xc5, 0xde, 0x94, - 0x89, 0xd8, 0x80, 0x3e, 0xc1, 0x29, 0x9f, 0xf4, 0x26, 0x90, 0xc8, 0x80, 0x72, 0xd7, 0xb6, 0x7c, - 0xdd, 0xb4, 0xb0, 0xcb, 0x14, 0xa5, 0x55, 0x17, 0x65, 0x2e, 0x4b, 0x34, 0x6c, 0x50, 0x21, 0xf3, - 0xc9, 0x17, 0x65, 0x64, 0x1c, 0x7a, 0x17, 0xa6, 0xc3, 0xc6, 0x85, 0x29, 0xc9, 0xa8, 0x2e, 0xca, - 0xac, 0x89, 0x24, 0x6c, 0x33, 0x48, 0x5c, 0xf2, 0x45, 0x19, 0x09, 0x85, 0xfa, 0x50, 0x71, 0x6c, - 0xe3, 0x8e, 0xc5, 0xcb, 0x75, 0xbd, 0xd3, 0xc7, 0x7c, 0x0e, 0xbb, 0x34, 0x56, 0x90, 0x48, 0x54, - 0x2c, 0x51, 0x27, 0x79, 0xe5, 0xab, 0x67, 0x49, 0x2c, 0x7a, 0x07, 0xa6, 0xfa, 0xa4, 0x7f, 0x5b, - 0xdb, 0x77, 0x4c, 0x17, 0x1b, 0xea, 0x8b, 0x62, 0x37, 0x04, 0x0a, 0x96, 0x26, 0x45, 0x1e, 0xf9, - 0x7c, 0x5c, 0xc4, 0x10, 0xef, 0x0f, 0xf4, 0xfd, 0x56, 0x60, 0x79, 0x6b, 0xfb, 0xfc, 0xd2, 0x4f, - 0x5e, 0xe5, 0xfd, 0x0d, 0x99, 0x88, 0x79, 0x3f, 0xc1, 0x29, 0x7b, 0x3f, 0x81, 0x44, 0x37, 0xe8, - 0x57, 0x80, 0xb9, 0x84, 0x5d, 0x18, 0x5b, 0x18, 0x5b, 0x2d, 0xe6, 0x0d, 0x36, 0x70, 0xe1, 0x4f, - 0x92, 0xd0, 0x48, 0x02, 0xf7, 0x01, 0x7d, 0xed, 0x16, 0xf6, 0x03, 0xd7, 0xc2, 0x06, 0x6f, 0xa3, - 0xc6, 0x7d, 0x20, 0x51, 0x45, 0x3e, 0x90, 0xa0, 0x63, 0x3e, 0x90, 0xb0, 0xe8, 0x03, 0x98, 0x4b, - 0x5c, 0x7f, 0x61, 0xef, 0x51, 0x52, 0x1d, 0x42, 0xac, 0x2b, 0x28, 0x59, 0xc7, 0xab, 0x92, 0x21, - 0x69, 0x56, 0x6a, 0x21, 0xda, 0x7b, 0xba, 0xd5, 0x5b, 0xb7, 0x3b, 0x72, 0xcc, 0x4d, 0xa9, 0xb4, - 0x5f, 0x55, 0x50, 0x32, 0xed, 0x2a, 0x19, 0xb2, 0x76, 0x15, 0x45, 0x74, 0xd5, 0x85, 0x14, 0x31, - 0xd1, 0x95, 0x30, 0xd5, 0x55, 0x17, 0x46, 0x20, 0x5c, 0x75, 0x61, 0x00, 0xc5, 0x55, 0x17, 0x4e, - 0x59, 0x08, 0x87, 0x35, 0xf5, 0xb7, 0x60, 0x26, 0x91, 0x5e, 0xd0, 0x6b, 0x10, 0x5d, 0xa0, 0xb8, - 0x7d, 0xe0, 0x84, 0xb5, 0xab, 0x74, 0xe1, 0x82, 0xc0, 0x55, 0x17, 0x2e, 0x08, 0xbc, 0xfe, 0x49, - 0x06, 0x0a, 0x61, 0x44, 0x1d, 0x4b, 0x37, 0xb2, 0x02, 0xf9, 0x01, 0xf6, 0xe8, 0x25, 0x89, 0x54, - 0x5c, 0xd4, 0x70, 0x90, 0x58, 0xd4, 0x70, 0x90, 0x5c, 0x73, 0xa5, 0x1f, 0xaa, 0xe6, 0xca, 0x1c, - 0xba, 0xe6, 0xc2, 0xf4, 0x5c, 0x55, 0xc8, 0x8b, 0xe1, 0xe1, 0xc6, 0xfd, 0x93, 0x6d, 0x78, 0xea, - 0x2a, 0x32, 0x26, 0x4e, 0x5d, 0x45, 0x14, 0xda, 0x85, 0x93, 0xc2, 0x01, 0x0c, 0x1f, 0xbd, 0x91, - 0x0c, 0x55, 0x9e, 0x7c, 0x88, 0xdd, 0xa2, 0x54, 0x6c, 0x1f, 0xee, 0x26, 0xa0, 0x62, 0xd1, 0x9a, - 0xc4, 0x91, 0x90, 0x30, 0x70, 0x27, 0xe8, 0x6d, 0xf0, 0x65, 0xcf, 0xc7, 0x21, 0x21, 0xc2, 0xc5, - 0x90, 0x10, 0xe1, 0xf5, 0x7f, 0xa7, 0xa0, 0x2c, 0xbf, 0xef, 0xb1, 0x04, 0xc6, 0x0b, 0x50, 0xc4, - 0xfb, 0xa6, 0xdf, 0xee, 0xda, 0x06, 0xe6, 0x9d, 0x1b, 0xf5, 0x33, 0x01, 0x5e, 0xb6, 0x0d, 0xc9, - 0xcf, 0x21, 0x4c, 0x8c, 0xa6, 0xf4, 0xa1, 0xa2, 0x29, 0x9e, 0x74, 0x66, 0x0e, 0x31, 0xe9, 0x54, - 0xfa, 0xa9, 0x78, 0x3c, 0x7e, 0xaa, 0x7f, 0x9e, 0x82, 0x4a, 0x32, 0xed, 0x7e, 0x33, 0xb6, 0xa0, - 0xbc, 0x9b, 0xd2, 0x87, 0xde, 0x4d, 0xaf, 0xc3, 0x34, 0xa9, 0xcc, 0x74, 0xdf, 0xe7, 0x77, 0x2a, - 0x33, 0xb4, 0xb8, 0x62, 0xd9, 0x28, 0xb0, 0x56, 0x43, 0xb8, 0x94, 0x8d, 0x04, 0xf8, 0x58, 0xe8, - 0x66, 0x8f, 0x18, 0xba, 0x1f, 0xa5, 0x60, 0x7a, 0xd3, 0x36, 0x6e, 0xb3, 0xa2, 0xcd, 0xff, 0xa6, - 0xac, 0xe7, 0xe3, 0x4c, 0x69, 0xf5, 0x19, 0x98, 0x96, 0xaa, 0xb6, 0xfa, 0xc7, 0x2c, 0xce, 0xe4, - 0xcf, 0xd5, 0xff, 0xdf, 0xba, 0x94, 0x61, 0x4a, 0x2c, 0xff, 0xea, 0x4d, 0x98, 0x49, 0x54, 0x6b, - 0xe2, 0x0b, 0x68, 0x87, 0x79, 0x81, 0xfa, 0x02, 0xcc, 0xa9, 0xca, 0x98, 0xfa, 0x55, 0x98, 0x53, - 0x15, 0x18, 0x47, 0x57, 0xf0, 0x1a, 0x3f, 0xcb, 0x64, 0xa5, 0xc0, 0xd1, 0xf9, 0x7f, 0x1b, 0xf5, - 0xc3, 0xf1, 0x8d, 0xe4, 0x37, 0xa0, 0xe2, 0x84, 0x0f, 0x6d, 0xde, 0x75, 0xb1, 0x8d, 0x46, 0x7b, - 0x88, 0x08, 0xb7, 0x9e, 0x68, 0xbf, 0xca, 0x32, 0x46, 0x96, 0xc3, 0x3b, 0xb2, 0x9c, 0x42, 0x4e, - 0x2b, 0xd1, 0x9a, 0x95, 0x65, 0x0c, 0xed, 0xd1, 0xb2, 0xa4, 0xb1, 0x9d, 0x49, 0xdc, 0x81, 0x46, - 0x17, 0xa1, 0x40, 0x7f, 0xa0, 0x14, 0x77, 0xa7, 0xf4, 0x7d, 0x29, 0x4c, 0x12, 0x99, 0xe7, 0x20, - 0xf4, 0x22, 0x14, 0xa3, 0x6b, 0xd1, 0xfc, 0x5c, 0x92, 0xc5, 0x46, 0x08, 0x94, 0x62, 0x23, 0x04, - 0xf2, 0xc6, 0xf6, 0x27, 0x70, 0x7a, 0xe2, 0x85, 0xe8, 0xa3, 0xf4, 0xc9, 0x42, 0x87, 0x9a, 0x39, - 0x52, 0x87, 0xba, 0x0e, 0x0b, 0xea, 0x7b, 0xca, 0x82, 0xf6, 0xd4, 0xa1, 0x8f, 0x4e, 0xa7, 0xc4, - 0xdb, 0xc0, 0xe8, 0x02, 0x64, 0x1d, 0xdb, 0xee, 0x7b, 0xfc, 0x28, 0x9f, 0x0a, 0xa0, 0x00, 0x51, - 0x00, 0x05, 0x3c, 0xc4, 0x48, 0x20, 0x08, 0xa3, 0x2c, 0xbe, 0xaf, 0xfc, 0x18, 0xd6, 0xeb, 0x99, - 0x8b, 0x50, 0x08, 0x8f, 0x4b, 0x11, 0x40, 0xee, 0xad, 0x3b, 0x6b, 0x77, 0xd6, 0xae, 0x54, 0x4e, - 0xa0, 0x12, 0xe4, 0x37, 0xd7, 0x6e, 0x5e, 0xb9, 0x7e, 0xf3, 0x6a, 0x45, 0x23, 0x0f, 0xad, 0x3b, - 0x37, 0x6f, 0x92, 0x87, 0xd4, 0x33, 0x37, 0xc4, 0x2b, 0x58, 0xbc, 0x5e, 0x9a, 0x82, 0xc2, 0xaa, - 0xe3, 0xd0, 0x8d, 0xcb, 0x78, 0xd7, 0xf6, 0x4c, 0xb2, 0xdb, 0x2a, 0x1a, 0xca, 0x43, 0xfa, 0xd6, - 0xad, 0x8d, 0x4a, 0x0a, 0xcd, 0x41, 0xe5, 0x0a, 0xd6, 0x8d, 0xbe, 0x69, 0xe1, 0x30, 0x5b, 0x54, - 0xd2, 0xcd, 0x7b, 0x7f, 0xf9, 0x72, 0x49, 0xfb, 0xfc, 0xcb, 0x25, 0xed, 0x5f, 0x5f, 0x2e, 0x69, - 0x9f, 0x7c, 0xb5, 0x74, 0xe2, 0xf3, 0xaf, 0x96, 0x4e, 0xfc, 0xfd, 0xab, 0xa5, 0x13, 0x3f, 0xbe, - 0xd8, 0x33, 0xfd, 0x9d, 0xa0, 0xd3, 0xe8, 0xda, 0x03, 0xfe, 0xdb, 0x49, 0xc7, 0xb5, 0xc9, 0x26, - 0xe6, 0x4f, 0x2b, 0xc9, 0x1f, 0x55, 0xfe, 0x2e, 0x75, 0x66, 0x95, 0x3e, 0x6e, 0x32, 0xba, 0xc6, - 0x75, 0xbb, 0xc1, 0x00, 0xf4, 0x67, 0x74, 0x5e, 0x27, 0x47, 0x7f, 0x2e, 0xf7, 0xc2, 0x7f, 0x03, - 0x00, 0x00, 0xff, 0xff, 0x5a, 0x9c, 0x59, 0x2e, 0x8f, 0x39, 0x00, 0x00, + // 3606 bytes of a gzipped FileDescriptorProto + 0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xe4, 0x3b, 0x4d, 0x6f, 0x1b, 0xd7, + 0xb5, 0x1e, 0x7e, 0xf3, 0x50, 0x1f, 0xf4, 0xd5, 0x87, 0x69, 0x25, 0x16, 0x65, 0x3a, 0xef, 0xc5, + 0x0e, 0x12, 0xca, 0x71, 0x5e, 0x1e, 0xf2, 0xf1, 0x90, 0x40, 0xb4, 0x15, 0xdb, 0x8a, 0x65, 0x2b, + 0x94, 0x9d, 0xe7, 0xf7, 0x10, 0x80, 0x19, 0x72, 0xae, 0xa8, 0xb1, 0xc8, 0x99, 0xc9, 0x7c, 0x28, + 0x12, 0x10, 0xa0, 0x49, 0x91, 0x76, 0x9d, 0x4d, 0x81, 0xa2, 0x9b, 0x66, 0xd3, 0x45, 0x8a, 0x76, + 0xd9, 0x6e, 0xbb, 0xed, 0xa2, 0x28, 0xb2, 0x29, 0x50, 0x14, 0x0d, 0x51, 0x24, 0xe8, 0x86, 0x8b, + 0xfe, 0x86, 0xe2, 0x7e, 0xcc, 0xcc, 0xbd, 0xc3, 0x4b, 0x8b, 0x72, 0x2c, 0x23, 0x4d, 0x57, 0xd2, + 0x9c, 0xef, 0xb9, 0xe7, 0xdc, 0x33, 0xe7, 0x9c, 0x7b, 0x09, 0xe7, 0x9c, 0xbd, 0xee, 0xaa, 0xee, + 0xf6, 0x75, 0x43, 0xc7, 0xfb, 0xd8, 0xf2, 0xbd, 0x55, 0xf6, 0xa7, 0xee, 0xb8, 0xb6, 0x6f, 0xa3, + 0x29, 0x11, 0xb5, 0x54, 0xdb, 0x7b, 0xc5, 0xab, 0x9b, 0xf6, 0xaa, 0xee, 0x98, 0xab, 0x1d, 0xdb, + 0xc5, 0xab, 0xfb, 0x2f, 0xae, 0x76, 0xb1, 0x85, 0x5d, 0xdd, 0xc7, 0x06, 0xe3, 0x58, 0xba, 0x28, + 0xd0, 0x58, 0xd8, 0xff, 0xd0, 0x76, 0xf7, 0x4c, 0xab, 0xab, 0xa2, 0xac, 0x76, 0x6d, 0xbb, 0xdb, + 0xc3, 0xab, 0xf4, 0xa9, 0x1d, 0xec, 0xac, 0xfa, 0x66, 0x1f, 0x7b, 0xbe, 0xde, 0x77, 0x38, 0xc1, + 0x7f, 0xc5, 0xa2, 0xfa, 0x7a, 0x67, 0xd7, 0xb4, 0xb0, 0x7b, 0xb8, 0x4a, 0xed, 0x75, 0xcc, 0x55, + 0x17, 0x7b, 0x76, 0xe0, 0x76, 0xf0, 0x88, 0xd8, 0xd7, 0x4c, 0xcb, 0xc7, 0xae, 0xa5, 0xf7, 0x56, + 0xbd, 0xce, 0x2e, 0x36, 0x82, 0x1e, 0x76, 0xe3, 0xff, 0xec, 0xf6, 0x03, 0xdc, 0xf1, 0xbd, 0x11, + 0x00, 0xe3, 0xad, 0x7d, 0xb5, 0x00, 0xd3, 0xeb, 0xe4, 0x5d, 0xb7, 0xf1, 0x07, 0x01, 0xb6, 0x3a, + 0x18, 0x5d, 0x82, 0xec, 0x07, 0x01, 0x0e, 0x70, 0x45, 0x5b, 0xd1, 0x2e, 0x16, 0x1b, 0x73, 0xc3, + 0x41, 0x75, 0x96, 0x02, 0x9e, 0xb7, 0xfb, 0xa6, 0x8f, 0xfb, 0x8e, 0x7f, 0xd8, 0x64, 0x14, 0xe8, + 0x35, 0x98, 0x7a, 0x60, 0xb7, 0x5b, 0x1e, 0xf6, 0x5b, 0x96, 0xde, 0xc7, 0x95, 0x14, 0xe5, 0xa8, + 0x0c, 0x07, 0xd5, 0xf9, 0x07, 0x76, 0x7b, 0x1b, 0xfb, 0xb7, 0xf5, 0xbe, 0xc8, 0x06, 0x31, 0x14, + 0xbd, 0x00, 0xf9, 0xc0, 0xc3, 0x6e, 0xcb, 0x34, 0x2a, 0x69, 0xca, 0x36, 0x3f, 0x1c, 0x54, 0xcb, + 0x04, 0x74, 0xd3, 0x10, 0x58, 0x72, 0x0c, 0x82, 0x9e, 0x87, 0x5c, 0xd7, 0xb5, 0x03, 0xc7, 0xab, + 0x64, 0x56, 0xd2, 0x21, 0x35, 0x83, 0x88, 0xd4, 0x0c, 0x82, 0xee, 0x40, 0x8e, 0x39, 0xb0, 0x92, + 0x5d, 0x49, 0x5f, 0x2c, 0x5d, 0x39, 0x5f, 0x17, 0xbd, 0x5a, 0x97, 0x5e, 0x98, 0x3d, 0x31, 0x81, + 0x0c, 0x2f, 0x0a, 0xe4, 0x71, 0xf0, 0xdb, 0x39, 0xc8, 0x52, 0x3a, 0xf4, 0x36, 0xe4, 0x3b, 0x2e, + 0x26, 0xab, 0x5f, 0x41, 0x2b, 0xda, 0xc5, 0xd2, 0x95, 0xa5, 0x3a, 0xf3, 0x6a, 0x3d, 0xf4, 0x6a, + 0xfd, 0x6e, 0xe8, 0xd5, 0xc6, 0xc2, 0x70, 0x50, 0x3d, 0xcd, 0xc9, 0x05, 0xa9, 0xa1, 0x04, 0xb4, + 0x05, 0x45, 0x2f, 0x68, 0xf7, 0x4d, 0x7f, 0xc3, 0x6e, 0xd3, 0xf5, 0x2e, 0x5d, 0x39, 0x23, 0x9b, + 0xba, 0x1d, 0xa2, 0x1b, 0x67, 0x86, 0x83, 0xea, 0x5c, 0x44, 0x1d, 0x4b, 0xbb, 0x71, 0xaa, 0x19, + 0x0b, 0x41, 0xbb, 0x30, 0xeb, 0x62, 0xc7, 0x35, 0x6d, 0xd7, 0xf4, 0x4d, 0x0f, 0x13, 0xb9, 0x29, + 0x2a, 0xf7, 0x9c, 0x2c, 0xb7, 0x29, 0x13, 0x35, 0xce, 0x0d, 0x07, 0xd5, 0xb3, 0x09, 0x4e, 0x49, + 0x47, 0x52, 0x2c, 0xf2, 0x01, 0x25, 0x40, 0xdb, 0xd8, 0xa7, 0xbe, 0x2c, 0x5d, 0x59, 0x79, 0xa8, + 0xb2, 0x6d, 0xec, 0x37, 0x56, 0x86, 0x83, 0xea, 0xd3, 0xa3, 0xfc, 0x92, 0x4a, 0x85, 0x7c, 0xd4, + 0x83, 0xb2, 0x08, 0x35, 0xc8, 0x0b, 0x66, 0xa8, 0xce, 0xe5, 0xf1, 0x3a, 0x09, 0x55, 0x63, 0x79, + 0x38, 0xa8, 0x2e, 0x25, 0x79, 0x25, 0x7d, 0x23, 0x92, 0x89, 0x7f, 0x3a, 0xba, 0xd5, 0xc1, 0x3d, + 0xa2, 0x26, 0xab, 0xf2, 0xcf, 0xd5, 0x10, 0xcd, 0xfc, 0x13, 0x51, 0xcb, 0xfe, 0x89, 0xc0, 0xe8, + 0x3d, 0x98, 0x8a, 0x1e, 0xc8, 0x7a, 0xe5, 0x78, 0x0c, 0xa9, 0x85, 0x92, 0x95, 0x5a, 0x1a, 0x0e, + 0xaa, 0x8b, 0x22, 0x8f, 0x24, 0x5a, 0x92, 0x16, 0x4b, 0xef, 0xb1, 0x95, 0xc9, 0x8f, 0x97, 0xce, + 0x28, 0x44, 0xe9, 0xbd, 0xd1, 0x15, 0x91, 0xa4, 0x11, 0xe9, 0x64, 0x03, 0x07, 0x9d, 0x0e, 0xc6, + 0x06, 0x36, 0x2a, 0x05, 0x95, 0xf4, 0x0d, 0x81, 0x82, 0x49, 0x17, 0x79, 0x64, 0xe9, 0x22, 0x86, + 0xac, 0xf5, 0x03, 0xbb, 0xbd, 0xee, 0xba, 0xb6, 0xeb, 0x55, 0x8a, 0xaa, 0xb5, 0xde, 0x08, 0xd1, + 0x6c, 0xad, 0x23, 0x6a, 0x79, 0xad, 0x23, 0x30, 0xb7, 0xb7, 0x19, 0x58, 0xb7, 0xb0, 0xee, 0x61, + 0xa3, 0x02, 0x63, 0xec, 0x8d, 0x28, 0x22, 0x7b, 0x23, 0xc8, 0x88, 0xbd, 0x11, 0x06, 0x19, 0x30, + 0xc3, 0x9e, 0xd7, 0x3c, 0xcf, 0xec, 0x5a, 0xd8, 0xa8, 0x94, 0xa8, 0xfc, 0xa7, 0x55, 0xf2, 0x43, + 0x9a, 0xc6, 0xd3, 0xc3, 0x41, 0xb5, 0x22, 0xf3, 0x49, 0x3a, 0x12, 0x32, 0xd1, 0xfb, 0x30, 0xcd, + 0x20, 0xcd, 0xc0, 0xb2, 0x4c, 0xab, 0x5b, 0x99, 0xa2, 0x4a, 0x9e, 0x52, 0x29, 0xe1, 0x24, 0x8d, + 0xa7, 0x86, 0x83, 0xea, 0x19, 0x89, 0x4b, 0x52, 0x21, 0x0b, 0x24, 0x19, 0x83, 0x01, 0x62, 0xc7, + 0x4e, 0xab, 0x32, 0xc6, 0x86, 0x4c, 0xc4, 0x32, 0x46, 0x82, 0x53, 0xce, 0x18, 0x09, 0x64, 0xec, + 0x0f, 0xee, 0xe4, 0x99, 0xf1, 0xfe, 0xe0, 0x7e, 0x16, 0xfc, 0xa1, 0x70, 0xb5, 0x24, 0x0d, 0x7d, + 0xac, 0xc1, 0x82, 0xe7, 0xeb, 0x96, 0xa1, 0xf7, 0x6c, 0x0b, 0xdf, 0xb4, 0xba, 0x2e, 0xf6, 0xbc, + 0x9b, 0xd6, 0x8e, 0x5d, 0x29, 0x53, 0x3d, 0x17, 0x12, 0x89, 0x55, 0x45, 0xda, 0xb8, 0x30, 0x1c, + 0x54, 0xab, 0x4a, 0x29, 0x92, 0x66, 0xb5, 0x22, 0x74, 0x00, 0x73, 0xe1, 0x47, 0xfa, 0x9e, 0x6f, + 0xf6, 0x4c, 0x4f, 0xf7, 0x4d, 0xdb, 0xaa, 0x9c, 0xa6, 0xfa, 0xcf, 0x27, 0xf3, 0xd3, 0x08, 0x61, + 0xe3, 0xfc, 0x70, 0x50, 0x3d, 0xa7, 0x90, 0x20, 0xe9, 0x56, 0xa9, 0x88, 0x9d, 0xb8, 0xe5, 0x62, + 0x42, 0x88, 0x8d, 0xca, 0xdc, 0x78, 0x27, 0x46, 0x44, 0xa2, 0x13, 0x23, 0xa0, 0xca, 0x89, 0x11, + 0x92, 0x68, 0x72, 0x74, 0xd7, 0x37, 0x89, 0xda, 0x4d, 0xdd, 0xdd, 0xc3, 0x6e, 0x65, 0x5e, 0xa5, + 0x69, 0x4b, 0x26, 0x62, 0x9a, 0x12, 0x9c, 0xb2, 0xa6, 0x04, 0x12, 0x7d, 0xa6, 0x81, 0x6c, 0x9a, + 0x69, 0x5b, 0x4d, 0xf2, 0xd1, 0xf6, 0xc8, 0xeb, 0x2d, 0x50, 0xa5, 0xcf, 0x3e, 0xe4, 0xf5, 0x44, + 0xf2, 0xc6, 0xb3, 0xc3, 0x41, 0xf5, 0xc2, 0x58, 0x69, 0x92, 0x21, 0xe3, 0x95, 0xa2, 0xfb, 0x50, + 0x22, 0x48, 0x4c, 0xcb, 0x1f, 0xa3, 0xb2, 0x48, 0x6d, 0x38, 0x3b, 0x6a, 0x03, 0x27, 0x68, 0x9c, + 0x1d, 0x0e, 0xaa, 0x0b, 0x02, 0x87, 0xa4, 0x47, 0x14, 0x85, 0x3e, 0xd5, 0x80, 0x04, 0xba, 0xea, + 0x4d, 0xcf, 0x50, 0x2d, 0xcf, 0x8c, 0x68, 0x51, 0xbd, 0xe6, 0x33, 0xc3, 0x41, 0x75, 0x45, 0x2d, + 0x47, 0xd2, 0x3d, 0x46, 0x57, 0x1c, 0x47, 0xd1, 0x47, 0xa2, 0x52, 0x19, 0x1f, 0x47, 0x11, 0x91, + 0x18, 0x47, 0x11, 0x50, 0x15, 0x47, 0x11, 0x92, 0x27, 0x83, 0x77, 0xf5, 0x9e, 0x69, 0xd0, 0x62, + 0xea, 0xec, 0x98, 0x64, 0x10, 0x51, 0x44, 0xc9, 0x20, 0x82, 0x8c, 0x24, 0x83, 0x98, 0x36, 0x0f, + 0x59, 0x2a, 0xa2, 0xf6, 0x97, 0x1c, 0xcc, 0x29, 0xb6, 0x1a, 0xc2, 0x30, 0x1d, 0xee, 0xa3, 0x96, + 0x49, 0x92, 0x44, 0x5a, 0xb5, 0xca, 0x6f, 0x07, 0x6d, 0xec, 0x5a, 0xd8, 0xc7, 0x5e, 0x28, 0x83, + 0x66, 0x09, 0x6a, 0x89, 0x2b, 0x40, 0x84, 0xda, 0x6e, 0x4a, 0x84, 0xa3, 0x9f, 0x69, 0x50, 0xe9, + 0xeb, 0x07, 0xad, 0x10, 0xe8, 0xb5, 0x76, 0x6c, 0xb7, 0xe5, 0x60, 0xd7, 0xb4, 0x0d, 0x5a, 0xc9, + 0x96, 0xae, 0xfc, 0xcf, 0x91, 0x79, 0xa1, 0xbe, 0xa9, 0x1f, 0x84, 0x60, 0xef, 0x2d, 0xdb, 0xdd, + 0xa2, 0xec, 0xeb, 0x96, 0xef, 0x1e, 0xb2, 0x84, 0xd5, 0x57, 0xe1, 0x05, 0x9b, 0x16, 0x94, 0x04, + 0xe8, 0x27, 0x1a, 0x2c, 0xfa, 0xb6, 0xaf, 0xf7, 0x5a, 0x9d, 0xa0, 0x1f, 0xf4, 0x74, 0xdf, 0xdc, + 0xc7, 0xad, 0xc0, 0xd3, 0xbb, 0x98, 0x97, 0xcd, 0xaf, 0x1f, 0x6d, 0xda, 0x5d, 0xc2, 0x7f, 0x35, + 0x62, 0xbf, 0x47, 0xb8, 0x99, 0x65, 0xb5, 0xe1, 0xa0, 0xba, 0xec, 0x2b, 0xd0, 0x82, 0x61, 0xf3, + 0x2a, 0x3c, 0x7a, 0x0e, 0x72, 0xa4, 0xad, 0x30, 0x0d, 0x5a, 0x1d, 0xf1, 0x16, 0xe4, 0x81, 0xdd, + 0x96, 0x1a, 0x83, 0x2c, 0x05, 0x10, 0x5a, 0x37, 0xb0, 0x08, 0x6d, 0x3e, 0xa6, 0x75, 0x03, 0x4b, + 0xa6, 0xa5, 0x80, 0xa5, 0xcf, 0x35, 0x58, 0x1a, 0xbf, 0x94, 0xe8, 0x02, 0xa4, 0xf7, 0xf0, 0x21, + 0x6f, 0x7b, 0x4e, 0x0f, 0x07, 0xd5, 0xe9, 0x3d, 0x7c, 0x28, 0x48, 0x21, 0x58, 0xf4, 0x7f, 0x90, + 0xdd, 0xd7, 0x7b, 0x01, 0xe6, 0x55, 0x75, 0xbd, 0xce, 0x3a, 0xb6, 0xba, 0xd8, 0xb1, 0xd5, 0x9d, + 0xbd, 0x2e, 0x01, 0xd4, 0x43, 0xaf, 0xd7, 0xdf, 0x09, 0x74, 0xcb, 0x37, 0xfd, 0x43, 0x66, 0x1e, + 0x15, 0x20, 0x9a, 0x47, 0x01, 0xaf, 0xa5, 0x5e, 0xd1, 0x96, 0x7e, 0xae, 0xc1, 0xd9, 0xb1, 0x4b, + 0xfa, 0x5d, 0xb0, 0x70, 0x23, 0x53, 0xd0, 0xca, 0xa9, 0x8d, 0x4c, 0x21, 0x55, 0x4e, 0xd7, 0x7e, + 0x95, 0x83, 0x62, 0xd4, 0xa0, 0xa0, 0x1b, 0x50, 0x36, 0xb0, 0x11, 0x38, 0x3d, 0xb3, 0x43, 0x63, + 0x83, 0x38, 0x85, 0x75, 0x84, 0x34, 0x3b, 0x48, 0x38, 0xc9, 0x3d, 0xb3, 0x09, 0x14, 0xba, 0x02, + 0x05, 0x5e, 0x88, 0x1f, 0xd2, 0x7d, 0x39, 0xdd, 0x58, 0x1c, 0x0e, 0xaa, 0x28, 0x84, 0x09, 0xac, + 0x11, 0x1d, 0x6a, 0x02, 0xb0, 0xce, 0x76, 0x13, 0xfb, 0x3a, 0x6f, 0x09, 0x2a, 0x72, 0xfc, 0xde, + 0x89, 0xf0, 0xac, 0x47, 0x8d, 0xe9, 0xc5, 0x1e, 0x35, 0x86, 0xa2, 0xf7, 0x00, 0xfa, 0xba, 0x69, + 0x31, 0x3e, 0x5e, 0xff, 0xd7, 0xc6, 0x65, 0x88, 0xcd, 0x88, 0x92, 0x49, 0x8f, 0x39, 0x45, 0xe9, + 0x31, 0x14, 0xdd, 0x81, 0x3c, 0xef, 0xc5, 0x2b, 0x39, 0xba, 0xdd, 0x96, 0xc7, 0x89, 0xe6, 0x62, + 0x69, 0x37, 0xc9, 0x59, 0xc4, 0x6e, 0x92, 0x83, 0xc8, 0xb2, 0xf5, 0xcc, 0x1d, 0xec, 0x9b, 0x7d, + 0x4c, 0x77, 0x03, 0x5f, 0xb6, 0x10, 0x26, 0x2e, 0x5b, 0x08, 0x43, 0xaf, 0x00, 0xe8, 0xfe, 0xa6, + 0xed, 0xf9, 0x77, 0xac, 0x0e, 0xa6, 0x15, 0x7d, 0x81, 0x99, 0x1f, 0x43, 0x45, 0xf3, 0x63, 0x28, + 0x7a, 0x1d, 0x4a, 0x0e, 0xff, 0x82, 0xb4, 0x7b, 0x98, 0x56, 0xec, 0x05, 0xf6, 0xc1, 0x13, 0xc0, + 0x02, 0xaf, 0x48, 0x8d, 0xae, 0xc3, 0x6c, 0xc7, 0xb6, 0x3a, 0x81, 0xeb, 0x62, 0xab, 0x73, 0xb8, + 0xad, 0xef, 0x60, 0x5a, 0x9d, 0x17, 0x58, 0xa8, 0x24, 0x50, 0x62, 0xa8, 0x24, 0x50, 0xe8, 0x65, + 0x28, 0x46, 0x93, 0x0d, 0x5a, 0x80, 0x17, 0x79, 0xa3, 0x1c, 0x02, 0x05, 0xe6, 0x98, 0x92, 0x18, + 0x6f, 0x7a, 0xd7, 0x78, 0xd0, 0x61, 0x5a, 0x54, 0x73, 0xe3, 0x05, 0xb0, 0x68, 0xbc, 0x00, 0x16, + 0xf2, 0xd3, 0xcc, 0x51, 0xf9, 0x29, 0xda, 0x2e, 0xd3, 0xe5, 0x99, 0xda, 0x1f, 0x34, 0x98, 0x57, + 0xc5, 0x4b, 0x22, 0x76, 0xb5, 0xc7, 0x12, 0xbb, 0xef, 0x42, 0xc1, 0xb1, 0x8d, 0x96, 0xe7, 0xe0, + 0x0e, 0xcf, 0x04, 0x89, 0xc8, 0xdd, 0xb2, 0x8d, 0x6d, 0x07, 0x77, 0xfe, 0xd7, 0xf4, 0x77, 0xd7, + 0xf6, 0x6d, 0xd3, 0xb8, 0x65, 0x7a, 0x3c, 0xc4, 0x1c, 0x86, 0x91, 0x3e, 0xaf, 0x79, 0x0e, 0x6c, + 0x14, 0x20, 0xc7, 0xb4, 0xd4, 0xfe, 0x98, 0x86, 0x72, 0x32, 0x46, 0xff, 0x95, 0x5e, 0x05, 0xdd, + 0x87, 0xbc, 0xc9, 0xaa, 0x77, 0xfe, 0xf5, 0xff, 0x0f, 0x21, 0x57, 0xd6, 0xe3, 0x51, 0x5e, 0x7d, + 0xff, 0xc5, 0x3a, 0x2f, 0xf3, 0xe9, 0x12, 0x50, 0xc9, 0x9c, 0x53, 0x96, 0xcc, 0x81, 0xa8, 0x09, + 0x79, 0x0f, 0xbb, 0xfb, 0x66, 0x07, 0xf3, 0x4c, 0x54, 0x15, 0x25, 0x77, 0x6c, 0x17, 0x13, 0x99, + 0xdb, 0x8c, 0x24, 0x96, 0xc9, 0x79, 0x64, 0x99, 0x1c, 0x88, 0xde, 0x85, 0x62, 0xc7, 0xb6, 0x76, + 0xcc, 0xee, 0xa6, 0xee, 0xf0, 0x5c, 0x74, 0x4e, 0x25, 0xf5, 0x6a, 0x48, 0xc4, 0x27, 0x12, 0xe1, + 0x63, 0x62, 0x22, 0x11, 0x51, 0xc5, 0x0e, 0xfd, 0x47, 0x06, 0x20, 0x76, 0x0e, 0x7a, 0x15, 0x4a, + 0xf8, 0x00, 0x77, 0x02, 0xdf, 0xa6, 0x53, 0x3a, 0x2d, 0x1e, 0xee, 0x85, 0x60, 0x29, 0xe0, 0x21, + 0x86, 0x92, 0x5d, 0x69, 0xe9, 0x7d, 0xec, 0x39, 0x7a, 0x27, 0x9c, 0x0a, 0x52, 0x63, 0x22, 0xa0, + 0xb8, 0x2b, 0x23, 0x20, 0xfa, 0x4f, 0xc8, 0xd0, 0x39, 0x22, 0x1b, 0x08, 0xa2, 0xe1, 0xa0, 0x3a, + 0x63, 0xc9, 0x13, 0x44, 0x8a, 0x47, 0x6f, 0xc2, 0xf4, 0x5e, 0x14, 0x78, 0xc4, 0xb6, 0x0c, 0x65, + 0xa0, 0x65, 0x59, 0x8c, 0x90, 0xac, 0x9b, 0x12, 0xe1, 0x68, 0x07, 0x4a, 0xba, 0x65, 0xd9, 0x3e, + 0xfd, 0xe0, 0x84, 0x43, 0xc2, 0x4b, 0xe3, 0xc2, 0xb4, 0xbe, 0x16, 0xd3, 0xb2, 0xda, 0x86, 0x66, + 0x0a, 0x41, 0x82, 0x98, 0x29, 0x04, 0x30, 0x6a, 0x42, 0xae, 0xa7, 0xb7, 0x71, 0x2f, 0xcc, 0xf0, + 0xcf, 0x8c, 0x55, 0x71, 0x8b, 0x92, 0x31, 0xe9, 0x74, 0x14, 0xc9, 0xf8, 0xc4, 0x51, 0x24, 0x83, + 0x2c, 0xed, 0x40, 0x39, 0x69, 0xcf, 0x64, 0x85, 0xc1, 0x25, 0xb1, 0x30, 0x28, 0x1e, 0x59, 0x8a, + 0xe8, 0x50, 0x12, 0x8c, 0x3a, 0x09, 0x15, 0xb5, 0x2f, 0x34, 0x98, 0x57, 0xed, 0x5d, 0xb4, 0x29, + 0xec, 0x78, 0x8d, 0x0f, 0x3c, 0x14, 0xa1, 0xce, 0x79, 0xc7, 0x6c, 0xf5, 0x78, 0xa3, 0x37, 0x60, + 0xc6, 0xb2, 0x0d, 0xdc, 0xd2, 0x89, 0x82, 0x9e, 0xe9, 0xf9, 0x95, 0x14, 0x1d, 0x22, 0xd3, 0x41, + 0x09, 0xc1, 0xac, 0x85, 0x08, 0x81, 0x7b, 0x5a, 0x42, 0xd4, 0x3e, 0x84, 0xd9, 0xc4, 0x18, 0x53, + 0x2a, 0x53, 0x52, 0x13, 0x96, 0x29, 0xf1, 0xb7, 0x23, 0x3d, 0xd9, 0xb7, 0xa3, 0xf6, 0xa3, 0x14, + 0x94, 0x84, 0x9e, 0x12, 0x3d, 0x80, 0x59, 0xfe, 0x1d, 0x33, 0xad, 0x2e, 0xeb, 0x5d, 0x52, 0x7c, + 0xc0, 0x31, 0x32, 0xe3, 0xdf, 0xb0, 0xdb, 0xdb, 0x11, 0x2d, 0x6d, 0x5d, 0xe8, 0xfc, 0xc9, 0x93, + 0x60, 0x82, 0xe2, 0x19, 0x19, 0x83, 0xee, 0xc3, 0x62, 0xe0, 0x90, 0x8e, 0xaa, 0xe5, 0xf1, 0x69, + 0x79, 0xcb, 0x0a, 0xfa, 0x6d, 0xec, 0x52, 0xeb, 0xb3, 0xac, 0xc6, 0x67, 0x14, 0xe1, 0x38, 0xfd, + 0x36, 0xc5, 0x8b, 0x35, 0xbe, 0x0a, 0x2f, 0xac, 0x43, 0x66, 0xc2, 0x75, 0xb8, 0x01, 0x68, 0x74, + 0x8e, 0x2c, 0xf9, 0x40, 0x9b, 0xcc, 0x07, 0xb5, 0x03, 0x28, 0x27, 0xa7, 0xc3, 0x4f, 0xc8, 0x97, + 0x7b, 0x50, 0x8c, 0x66, 0xbb, 0xe8, 0x79, 0xc8, 0xb9, 0x58, 0xf7, 0x6c, 0x8b, 0xef, 0x16, 0xba, + 0xed, 0x19, 0x44, 0xdc, 0xf6, 0x0c, 0xf2, 0x08, 0xca, 0xee, 0xc2, 0x14, 0x5b, 0xa4, 0xb7, 0xcc, + 0x9e, 0x8f, 0x5d, 0x74, 0x0d, 0x72, 0x9e, 0xaf, 0xfb, 0xd8, 0xab, 0x68, 0x2b, 0xe9, 0x8b, 0x33, + 0x57, 0x16, 0x47, 0x07, 0xb7, 0x04, 0xcd, 0xec, 0x60, 0x94, 0xa2, 0x1d, 0x0c, 0x52, 0xfb, 0xa1, + 0x06, 0x53, 0xe2, 0x7c, 0xfa, 0xf1, 0x88, 0x3d, 0xde, 0x62, 0xd4, 0xac, 0xd0, 0x06, 0x3e, 0x99, + 0x3e, 0xe9, 0xa5, 0xfc, 0x42, 0x63, 0x6b, 0x19, 0x8d, 0x32, 0xbb, 0xf1, 0xf8, 0x80, 0x6c, 0x14, + 0x8f, 0x26, 0x94, 0x49, 0xc7, 0x07, 0x34, 0xed, 0x48, 0xec, 0x62, 0xda, 0x91, 0x10, 0x8f, 0x60, + 0xeb, 0xe7, 0x59, 0x6a, 0x6b, 0x3c, 0xa8, 0x4e, 0x7c, 0xc7, 0xd3, 0xc7, 0xf8, 0x8e, 0xbf, 0x00, + 0x79, 0x9a, 0x38, 0xa3, 0x6d, 0x4a, 0x17, 0x96, 0x80, 0xe4, 0x43, 0x3a, 0x06, 0x79, 0x48, 0xba, + 0xc8, 0x7e, 0xcb, 0x74, 0xd1, 0x82, 0xb3, 0xbb, 0xba, 0xd7, 0x0a, 0x13, 0x9c, 0xd1, 0xd2, 0xfd, + 0x56, 0xb4, 0x5f, 0x73, 0xb4, 0x7a, 0xa7, 0xa3, 0xaf, 0x5d, 0xdd, 0xdb, 0x0e, 0x69, 0xd6, 0xfc, + 0xad, 0xd1, 0xdd, 0xbb, 0xa8, 0xa6, 0x40, 0xf7, 0x60, 0x41, 0x2d, 0x3c, 0x4f, 0x2d, 0xa7, 0x93, + 0x59, 0xef, 0xa1, 0x92, 0xe7, 0x14, 0x68, 0xf4, 0x89, 0x06, 0x15, 0xf2, 0x25, 0x73, 0xf1, 0x07, + 0x81, 0xe9, 0xe2, 0x3e, 0x09, 0x8b, 0x96, 0xbd, 0x8f, 0xdd, 0x9e, 0x7e, 0xc8, 0x0f, 0x39, 0xce, + 0x8f, 0xa6, 0xed, 0x2d, 0xdb, 0x68, 0x0a, 0x0c, 0xec, 0xd5, 0x1c, 0x19, 0x78, 0x87, 0x09, 0x11, + 0x5f, 0x4d, 0x4d, 0x21, 0x84, 0x10, 0x1c, 0x63, 0x9c, 0x52, 0x3a, 0x6a, 0x9c, 0x42, 0xaa, 0x35, + 0xc7, 0xb6, 0x7b, 0xb4, 0x79, 0xe2, 0xd5, 0x1a, 0x79, 0x16, 0xab, 0x35, 0xf2, 0x2c, 0x4e, 0x0c, + 0x36, 0x32, 0x85, 0x42, 0xb9, 0x58, 0xfb, 0x4a, 0x83, 0x19, 0xf9, 0x5c, 0x64, 0x74, 0x43, 0xa5, + 0x4f, 0x7c, 0x43, 0x65, 0x8e, 0xb1, 0x1a, 0xd9, 0xa3, 0x56, 0x43, 0x9a, 0x8b, 0xfc, 0x55, 0x83, + 0x69, 0xe9, 0x48, 0xe6, 0xfb, 0xf5, 0x7a, 0x3f, 0x4d, 0xc1, 0xa2, 0xda, 0xd4, 0x13, 0x69, 0xff, + 0x6e, 0x00, 0x29, 0xe4, 0x6e, 0xc6, 0x85, 0xce, 0xc2, 0x48, 0xf7, 0x47, 0x97, 0x29, 0xac, 0x02, + 0x47, 0x4e, 0x6b, 0x42, 0x76, 0x74, 0x1f, 0x4a, 0xa6, 0x70, 0x2e, 0x94, 0x56, 0x8d, 0xef, 0xc5, + 0xd3, 0x20, 0x36, 0x10, 0x18, 0x73, 0x06, 0x24, 0x8a, 0x6a, 0xe4, 0x20, 0x43, 0x2a, 0xb1, 0xda, + 0x3e, 0xe4, 0xb9, 0x39, 0xe8, 0x25, 0x28, 0xd2, 0xdc, 0x49, 0x3b, 0x1a, 0x56, 0x36, 0xd3, 0x92, + 0x82, 0x00, 0x13, 0xf7, 0x22, 0x0a, 0x21, 0x0c, 0xfd, 0x37, 0x00, 0x49, 0x17, 0x3c, 0x6b, 0xa6, + 0x68, 0xee, 0xa1, 0x9d, 0x93, 0x63, 0x1b, 0x23, 0xa9, 0xb2, 0x18, 0x01, 0x6b, 0xbf, 0x4e, 0x41, + 0x49, 0x3c, 0x89, 0x7a, 0x24, 0xe5, 0x1f, 0x41, 0xd8, 0xd5, 0xb6, 0x74, 0xc3, 0x20, 0x7f, 0x71, + 0xf8, 0x61, 0x5b, 0x1d, 0xbb, 0x48, 0xe1, 0xff, 0x6b, 0x21, 0x07, 0xeb, 0x61, 0xe8, 0x69, 0xbb, + 0x99, 0x40, 0x09, 0x5a, 0xcb, 0x49, 0xdc, 0xd2, 0x1e, 0x2c, 0x28, 0x45, 0x89, 0x9d, 0x47, 0xf6, + 0x71, 0x75, 0x1e, 0xbf, 0xc8, 0xc2, 0x82, 0xf2, 0x04, 0x30, 0x11, 0xc1, 0xe9, 0xc7, 0x12, 0xc1, + 0x3f, 0xd6, 0x54, 0x2b, 0xcb, 0xc6, 0xff, 0xaf, 0x4e, 0x70, 0x2c, 0xf9, 0xb8, 0xd6, 0x58, 0x0e, + 0x8b, 0xec, 0x23, 0xc5, 0x64, 0x6e, 0xd2, 0x98, 0x44, 0x97, 0x59, 0x13, 0x47, 0x75, 0xb1, 0xe1, + 0x7c, 0xb8, 0x43, 0x13, 0xaa, 0xf2, 0x1c, 0x44, 0xfa, 0xfa, 0x90, 0x83, 0x8d, 0x0e, 0x0a, 0x71, + 0x5f, 0xcf, 0x69, 0x92, 0xd3, 0x83, 0x29, 0x11, 0x2e, 0x64, 0xbf, 0xe2, 0x31, 0xb2, 0x1f, 0x1c, + 0x79, 0x72, 0xf0, 0x24, 0x63, 0x53, 0x4a, 0xb5, 0x03, 0x0d, 0x66, 0x13, 0x07, 0xef, 0xdf, 0xaf, + 0x6f, 0xc9, 0xc7, 0x1a, 0x14, 0xa3, 0x7b, 0x1d, 0x68, 0x0d, 0x72, 0x98, 0xdd, 0x0d, 0x60, 0x69, + 0x67, 0x2e, 0x71, 0x6f, 0x8b, 0xe0, 0xf8, 0x4d, 0xad, 0xc4, 0x75, 0x80, 0x26, 0x67, 0x7c, 0x84, + 0x82, 0xf9, 0x37, 0x5a, 0x58, 0x30, 0x8f, 0x58, 0x91, 0xfe, 0xf6, 0x56, 0x9c, 0xdc, 0xd2, 0xfd, + 0xae, 0x08, 0x59, 0x6a, 0x0b, 0x69, 0x5e, 0x7d, 0xec, 0xf6, 0x4d, 0x4b, 0xef, 0xd1, 0x50, 0x2c, + 0xb0, 0x5d, 0x1d, 0xc2, 0xc4, 0x5d, 0x1d, 0xc2, 0xd0, 0x2e, 0xcc, 0xc6, 0x23, 0x31, 0x2a, 0x46, + 0x7d, 0x51, 0xec, 0x6d, 0x99, 0x88, 0x0d, 0xe8, 0x13, 0x9c, 0xf2, 0x49, 0x6f, 0x02, 0x89, 0x0c, + 0x98, 0xe9, 0xd8, 0x96, 0xaf, 0x9b, 0x16, 0x76, 0x99, 0xa2, 0xb4, 0xea, 0xa2, 0xcc, 0x55, 0x89, + 0x86, 0x0d, 0x2a, 0x64, 0x3e, 0xf9, 0xa2, 0x8c, 0x8c, 0x43, 0xef, 0xc3, 0x74, 0xd8, 0xb8, 0x30, + 0x25, 0x19, 0xd5, 0x45, 0x99, 0x75, 0x91, 0x84, 0x6d, 0x06, 0x89, 0x4b, 0xbe, 0x28, 0x23, 0xa1, + 0x50, 0x0f, 0xca, 0x8e, 0x6d, 0xdc, 0xb3, 0x78, 0xb9, 0xae, 0xb7, 0x7b, 0x98, 0xcf, 0x61, 0x97, + 0x47, 0x0a, 0x12, 0x89, 0x8a, 0x25, 0xea, 0x24, 0xaf, 0x7c, 0xf5, 0x2c, 0x89, 0x45, 0xef, 0xc1, + 0x54, 0x8f, 0xf4, 0x6f, 0xeb, 0x07, 0x8e, 0xe9, 0x62, 0x43, 0x7d, 0x51, 0xec, 0x96, 0x40, 0xc1, + 0xd2, 0xa4, 0xc8, 0x23, 0x9f, 0x8f, 0x8b, 0x18, 0xe2, 0xfd, 0xbe, 0x7e, 0xd0, 0x0c, 0x2c, 0x6f, + 0xfd, 0x80, 0x5f, 0xfa, 0xc9, 0xab, 0xbc, 0xbf, 0x29, 0x13, 0x31, 0xef, 0x27, 0x38, 0x65, 0xef, + 0x27, 0x90, 0xe8, 0x16, 0xfd, 0x0a, 0x30, 0x97, 0xb0, 0x0b, 0x63, 0x8b, 0x23, 0xab, 0xc5, 0xbc, + 0xc1, 0x06, 0x2e, 0xfc, 0x49, 0x12, 0x1a, 0x49, 0xe0, 0x3e, 0xa0, 0xaf, 0xdd, 0xc4, 0x7e, 0xe0, + 0x5a, 0xd8, 0xe0, 0x6d, 0xd4, 0xa8, 0x0f, 0x24, 0xaa, 0xc8, 0x07, 0x12, 0x74, 0xc4, 0x07, 0x12, + 0x16, 0x7d, 0x04, 0xf3, 0x89, 0xeb, 0x2f, 0xec, 0x3d, 0x4a, 0xaa, 0x43, 0x88, 0x0d, 0x05, 0x25, + 0xeb, 0x78, 0x55, 0x32, 0x24, 0xcd, 0x4a, 0x2d, 0x44, 0x7b, 0x57, 0xb7, 0xba, 0x1b, 0x76, 0x5b, + 0x8e, 0xb9, 0x29, 0x95, 0xf6, 0xeb, 0x0a, 0x4a, 0xa6, 0x5d, 0x25, 0x43, 0xd6, 0xae, 0xa2, 0x88, + 0xae, 0xba, 0x90, 0x22, 0x26, 0xba, 0x12, 0xa6, 0xba, 0xea, 0xc2, 0x08, 0x84, 0xab, 0x2e, 0x0c, + 0xa0, 0xb8, 0xea, 0xc2, 0x29, 0x0b, 0xe1, 0xb0, 0xa6, 0xf6, 0x0e, 0xcc, 0x26, 0xd2, 0x0b, 0x7a, + 0x03, 0xa2, 0x0b, 0x14, 0x77, 0x0f, 0x9d, 0xb0, 0x76, 0x95, 0x2e, 0x5c, 0x10, 0xb8, 0xea, 0xc2, + 0x05, 0x81, 0xd7, 0x3e, 0xcb, 0x40, 0x21, 0x8c, 0xa8, 0x13, 0xe9, 0x46, 0x56, 0x21, 0xdf, 0xc7, + 0x1e, 0xbd, 0x24, 0x91, 0x8a, 0x8b, 0x1a, 0x0e, 0x12, 0x8b, 0x1a, 0x0e, 0x92, 0x6b, 0xae, 0xf4, + 0x23, 0xd5, 0x5c, 0x99, 0x89, 0x6b, 0x2e, 0x4c, 0xcf, 0x55, 0x85, 0xbc, 0x18, 0x1e, 0x6e, 0x3c, + 0x3c, 0xd9, 0x86, 0xa7, 0xae, 0x22, 0x63, 0xe2, 0xd4, 0x55, 0x44, 0xa1, 0x3d, 0x38, 0x2d, 0x1c, + 0xc0, 0xf0, 0xd1, 0x1b, 0xc9, 0x50, 0x33, 0xe3, 0x0f, 0xb1, 0x9b, 0x94, 0x8a, 0xed, 0xc3, 0xbd, + 0x04, 0x54, 0x2c, 0x5a, 0x93, 0x38, 0x12, 0x12, 0x06, 0x6e, 0x07, 0xdd, 0x4d, 0xbe, 0xec, 0xf9, + 0x38, 0x24, 0x44, 0xb8, 0x18, 0x12, 0x22, 0xbc, 0xf6, 0xf7, 0x14, 0xcc, 0xc8, 0xef, 0x7b, 0x22, + 0x81, 0xf1, 0x12, 0x14, 0xf1, 0x81, 0xe9, 0xb7, 0x3a, 0xb6, 0x81, 0x79, 0xe7, 0x46, 0xfd, 0x4c, + 0x80, 0x57, 0x6d, 0x43, 0xf2, 0x73, 0x08, 0x13, 0xa3, 0x29, 0x3d, 0x51, 0x34, 0xc5, 0x93, 0xce, + 0xcc, 0x04, 0x93, 0x4e, 0xa5, 0x9f, 0x8a, 0x27, 0xe3, 0xa7, 0xda, 0x97, 0x29, 0x28, 0x27, 0xd3, + 0xee, 0x77, 0x63, 0x0b, 0xca, 0xbb, 0x29, 0x3d, 0xf1, 0x6e, 0x7a, 0x13, 0xa6, 0x49, 0x65, 0xa6, + 0xfb, 0x3e, 0xbf, 0x53, 0x99, 0xa1, 0xc5, 0x15, 0xcb, 0x46, 0x81, 0xb5, 0x16, 0xc2, 0xa5, 0x6c, + 0x24, 0xc0, 0x47, 0x42, 0x37, 0x7b, 0xcc, 0xd0, 0xfd, 0x24, 0x05, 0xd3, 0x5b, 0xb6, 0x71, 0x97, + 0x15, 0x6d, 0xfe, 0x77, 0x65, 0x3d, 0x9f, 0x64, 0x4a, 0xab, 0xcd, 0xc2, 0xb4, 0x54, 0xb5, 0xd5, + 0x3e, 0x65, 0x71, 0x26, 0x7f, 0xae, 0xfe, 0xfd, 0xd6, 0x65, 0x06, 0xa6, 0xc4, 0xf2, 0xaf, 0xd6, + 0x80, 0xd9, 0x44, 0xb5, 0x26, 0xbe, 0x80, 0x36, 0xc9, 0x0b, 0xd4, 0xae, 0xc1, 0xbc, 0xaa, 0x8c, + 0x11, 0xb2, 0x8e, 0x36, 0xc1, 0xe9, 0xcc, 0x75, 0x98, 0x57, 0x95, 0x23, 0xc7, 0x37, 0xe7, 0x0d, + 0x7e, 0xf2, 0xc9, 0x0a, 0x87, 0xe3, 0xf3, 0xff, 0x29, 0xea, 0x9e, 0xe3, 0xfb, 0xcb, 0x6f, 0x41, + 0xd9, 0x09, 0x1f, 0x5a, 0xbc, 0x47, 0x63, 0xdb, 0x92, 0x76, 0x1c, 0x11, 0x6e, 0x23, 0xd1, 0xac, + 0xcd, 0xc8, 0x18, 0x59, 0x0e, 0xef, 0xdf, 0x72, 0x0a, 0x39, 0xcd, 0x44, 0x23, 0x37, 0x23, 0x63, + 0x84, 0xa5, 0xcd, 0x1f, 0xbd, 0xb4, 0xb4, 0xff, 0xcb, 0x92, 0xa6, 0x79, 0x36, 0x71, 0xbf, 0x1a, + 0x5d, 0x86, 0x02, 0xfd, 0xf1, 0x53, 0xdc, 0xf9, 0xd2, 0xd5, 0xa1, 0x30, 0xc9, 0x80, 0x3c, 0x07, + 0xa1, 0x97, 0xa1, 0x18, 0x5d, 0xb9, 0xe6, 0x67, 0x9e, 0x2c, 0xee, 0x42, 0xa0, 0x14, 0x77, 0x21, + 0x90, 0x37, 0xcd, 0x3f, 0x80, 0xb3, 0x63, 0x2f, 0x5b, 0x1f, 0xa7, 0x07, 0x17, 0xba, 0xdf, 0xcc, + 0xb1, 0xba, 0xdf, 0x03, 0x58, 0x54, 0xdf, 0x81, 0x16, 0xb4, 0xa7, 0x8e, 0xd4, 0x1e, 0xaf, 0x7e, + 0x7a, 0xc2, 0xd5, 0x4f, 0xd5, 0xf6, 0xe8, 0xb8, 0x20, 0xba, 0x6b, 0x8c, 0x2e, 0x41, 0xd6, 0xb1, + 0xed, 0x9e, 0xc7, 0x2f, 0x15, 0x50, 0x75, 0x14, 0x20, 0xaa, 0xa3, 0x80, 0x47, 0x18, 0x4e, 0x04, + 0x61, 0x04, 0xc7, 0x37, 0xa7, 0x9f, 0xc0, 0xea, 0x3e, 0x77, 0x19, 0x0a, 0xe1, 0xc1, 0x2d, 0x02, + 0xc8, 0xbd, 0x73, 0x6f, 0xfd, 0xde, 0xfa, 0xb5, 0xf2, 0x29, 0x54, 0x82, 0xfc, 0xd6, 0xfa, 0xed, + 0x6b, 0x37, 0x6f, 0x5f, 0x2f, 0x6b, 0xe4, 0xa1, 0x79, 0xef, 0xf6, 0x6d, 0xf2, 0x90, 0x7a, 0xee, + 0x96, 0x78, 0x19, 0x8c, 0x57, 0x6e, 0x53, 0x50, 0x58, 0x73, 0x1c, 0x9a, 0x42, 0x18, 0xef, 0xfa, + 0xbe, 0x49, 0x76, 0x72, 0x59, 0x43, 0x79, 0x48, 0xdf, 0xb9, 0xb3, 0x59, 0x4e, 0xa1, 0x79, 0x28, + 0x5f, 0xc3, 0xba, 0xd1, 0x33, 0x2d, 0x1c, 0xe6, 0xad, 0x72, 0xba, 0xf1, 0xe0, 0xf7, 0x5f, 0x2f, + 0x6b, 0x5f, 0x7e, 0xbd, 0xac, 0xfd, 0xed, 0xeb, 0x65, 0xed, 0xb3, 0x6f, 0x96, 0x4f, 0x7d, 0xf9, + 0xcd, 0xf2, 0xa9, 0x3f, 0x7f, 0xb3, 0x7c, 0xea, 0xff, 0x2f, 0x77, 0x4d, 0x7f, 0x37, 0x68, 0xd7, + 0x3b, 0x76, 0x9f, 0xff, 0x8a, 0xd3, 0x71, 0x6d, 0x92, 0x20, 0xf8, 0xd3, 0x6a, 0xf2, 0xe7, 0x9d, + 0xbf, 0x4c, 0x9d, 0x5b, 0xa3, 0x8f, 0x5b, 0x8c, 0xae, 0x7e, 0xd3, 0xae, 0x33, 0x00, 0xfd, 0x41, + 0x9f, 0xd7, 0xce, 0xd1, 0x1f, 0xee, 0xbd, 0xf4, 0xcf, 0x00, 0x00, 0x00, 0xff, 0xff, 0xfb, 0xb5, + 0x74, 0x30, 0x19, 0x3a, 0x00, 0x00, } func (m *EventSequence) Marshal() (dAtA []byte, err error) { @@ -6680,6 +6706,13 @@ func (m *JobRunPreemptedError) MarshalToSizedBuffer(dAtA []byte) (int, error) { _ = i var l int _ = l + if len(m.Reason) > 0 { + i -= len(m.Reason) + copy(dAtA[i:], m.Reason) + i = encodeVarintEvents(dAtA, i, uint64(len(m.Reason))) + i-- + dAtA[i] = 0xa + } return len(dAtA) - i, nil } @@ -6763,6 +6796,13 @@ func (m *JobRunPreempted) MarshalToSizedBuffer(dAtA []byte) (int, error) { _ = i var l int _ = l + if len(m.Reason) > 0 { + i -= len(m.Reason) + copy(dAtA[i:], m.Reason) + i = encodeVarintEvents(dAtA, i, uint64(len(m.Reason))) + i-- + dAtA[i] = 0x3a + } if len(m.PreemptedRunId) > 0 { i -= len(m.PreemptedRunId) copy(dAtA[i:], m.PreemptedRunId) @@ -6872,6 +6912,13 @@ func (m *JobPreemptionRequested) MarshalToSizedBuffer(dAtA []byte) (int, error) _ = i var l int _ = l + if len(m.Reason) > 0 { + i -= len(m.Reason) + copy(dAtA[i:], m.Reason) + i = encodeVarintEvents(dAtA, i, uint64(len(m.Reason))) + i-- + dAtA[i] = 0x1a + } if len(m.JobId) > 0 { i -= len(m.JobId) copy(dAtA[i:], m.JobId) @@ -8309,6 +8356,10 @@ func (m *JobRunPreemptedError) Size() (n int) { } var l int _ = l + l = len(m.Reason) + if l > 0 { + n += 1 + l + sovEvents(uint64(l)) + } return n } @@ -8352,6 +8403,10 @@ func (m *JobRunPreempted) Size() (n int) { if l > 0 { n += 1 + l + sovEvents(uint64(l)) } + l = len(m.Reason) + if l > 0 { + n += 1 + l + sovEvents(uint64(l)) + } return n } @@ -8398,6 +8453,10 @@ func (m *JobPreemptionRequested) Size() (n int) { if l > 0 { n += 1 + l + sovEvents(uint64(l)) } + l = len(m.Reason) + if l > 0 { + n += 1 + l + sovEvents(uint64(l)) + } return n } @@ -15802,6 +15861,38 @@ func (m *JobRunPreemptedError) Unmarshal(dAtA []byte) error { return fmt.Errorf("proto: JobRunPreemptedError: illegal tag %d (wire type %d)", fieldNum, wire) } switch fieldNum { + case 1: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Reason", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowEvents + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= uint64(b&0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthEvents + } + postIndex := iNdEx + intStringLen + if postIndex < 0 { + return ErrInvalidLengthEvents + } + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.Reason = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipEvents(dAtA[iNdEx:]) @@ -16080,6 +16171,38 @@ func (m *JobRunPreempted) Unmarshal(dAtA []byte) error { } m.PreemptedRunId = string(dAtA[iNdEx:postIndex]) iNdEx = postIndex + case 7: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Reason", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowEvents + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= uint64(b&0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthEvents + } + postIndex := iNdEx + intStringLen + if postIndex < 0 { + return ErrInvalidLengthEvents + } + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.Reason = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipEvents(dAtA[iNdEx:]) @@ -16377,6 +16500,38 @@ func (m *JobPreemptionRequested) Unmarshal(dAtA []byte) error { } m.JobId = string(dAtA[iNdEx:postIndex]) iNdEx = postIndex + case 3: + if wireType != 2 { + return fmt.Errorf("proto: wrong wireType = %d for field Reason", wireType) + } + var stringLen uint64 + for shift := uint(0); ; shift += 7 { + if shift >= 64 { + return ErrIntOverflowEvents + } + if iNdEx >= l { + return io.ErrUnexpectedEOF + } + b := dAtA[iNdEx] + iNdEx++ + stringLen |= uint64(b&0x7F) << shift + if b < 0x80 { + break + } + } + intStringLen := int(stringLen) + if intStringLen < 0 { + return ErrInvalidLengthEvents + } + postIndex := iNdEx + intStringLen + if postIndex < 0 { + return ErrInvalidLengthEvents + } + if postIndex > l { + return io.ErrUnexpectedEOF + } + m.Reason = string(dAtA[iNdEx:postIndex]) + iNdEx = postIndex default: iNdEx = preIndex skippy, err := skipEvents(dAtA[iNdEx:]) diff --git a/pkg/armadaevents/events.proto b/pkg/armadaevents/events.proto index d6019606490..4b14cf4b402 100644 --- a/pkg/armadaevents/events.proto +++ b/pkg/armadaevents/events.proto @@ -510,6 +510,7 @@ message MaxRunsExceeded { } message JobRunPreemptedError{ + string reason = 1; } message GangJobUnschedulable{ @@ -525,6 +526,7 @@ message JobRunPreempted{ reserved 1 to 4; string preempted_job_id = 5; string preempted_run_id = 6; + string reason = 7; } // Message used internally by Armada to see if messages can be propagated through a pulsar partition @@ -548,6 +550,7 @@ message JobRunPreemptionRequested { message JobPreemptionRequested { reserved 1; string job_id = 2; + string reason = 3; } // Indicates that the scheduler is happy with the job From 43f8573fabd0f922891e496e43b25a9e7ad8ee00 Mon Sep 17 00:00:00 2001 From: robertdavidsmith <34475852+robertdavidsmith@users.noreply.github.com> Date: Fri, 8 Nov 2024 14:50:02 +0000 Subject: [PATCH 2/2] Scheduler: move more to internaltypes.ResourceList (#4036) --- .../floating_resource_types.go | 36 +- .../floating_resource_types_test.go | 73 ++-- .../resource_fraction_list_test.go | 3 +- .../scheduler/internaltypes/resource_list.go | 64 +++- .../internaltypes/resource_list_map_util.go | 59 ++++ .../resource_list_map_util_test.go | 111 ++++++ .../internaltypes/resource_list_test.go | 87 ++++- .../scheduler/metrics/cycle_metrics_test.go | 13 +- internal/scheduler/nodedb/nodedb.go | 7 +- internal/scheduler/nodedb/nodedb_test.go | 13 +- internal/scheduler/schedulerapp.go | 2 +- .../scheduling/constraints/constraints.go | 237 ++++++------- .../constraints/constraints_test.go | 330 +++++++----------- internal/scheduler/scheduling/context/gang.go | 8 +- .../scheduler/scheduling/context/gang_test.go | 7 +- .../scheduler/scheduling/context/queue.go | 31 +- .../scheduling/context/scheduling.go | 51 +-- .../scheduling/context/scheduling_test.go | 39 +-- .../scheduler/scheduling/fairness/fairness.go | 67 ++-- .../scheduling/fairness/fairness_test.go | 42 ++- .../scheduler/scheduling/gang_scheduler.go | 2 +- .../scheduling/gang_scheduler_test.go | 28 +- .../scheduling/preempting_queue_scheduler.go | 15 +- .../preempting_queue_scheduler_test.go | 36 +- .../scheduler/scheduling/queue_scheduler.go | 22 +- .../scheduling/queue_scheduler_test.go | 21 +- internal/scheduler/scheduling/result.go | 6 +- .../scheduler/scheduling/scheduling_algo.go | 71 ++-- internal/scheduler/simulator/simulator.go | 45 +-- .../simulator/sink/queue_stats_writer.go | 6 +- internal/scheduler/simulator/test_utils.go | 14 - .../scheduler/testfixtures/testfixtures.go | 2 +- 32 files changed, 865 insertions(+), 683 deletions(-) create mode 100644 internal/scheduler/internaltypes/resource_list_map_util.go create mode 100644 internal/scheduler/internaltypes/resource_list_map_util_test.go diff --git a/internal/scheduler/floatingresources/floating_resource_types.go b/internal/scheduler/floatingresources/floating_resource_types.go index f4670d13925..3b023e0c891 100644 --- a/internal/scheduler/floatingresources/floating_resource_types.go +++ b/internal/scheduler/floatingresources/floating_resource_types.go @@ -9,19 +9,21 @@ import ( "github.com/armadaproject/armada/internal/common/maps" "github.com/armadaproject/armada/internal/scheduler/configuration" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" ) type FloatingResourceTypes struct { zeroFloatingResources schedulerobjects.ResourceList pools map[string]*floatingResourcePool + rlFactory *internaltypes.ResourceListFactory } type floatingResourcePool struct { totalResources schedulerobjects.ResourceList } -func NewFloatingResourceTypes(config []configuration.FloatingResourceConfig) (*FloatingResourceTypes, error) { +func NewFloatingResourceTypes(config []configuration.FloatingResourceConfig, rlFactory *internaltypes.ResourceListFactory) (*FloatingResourceTypes, error) { zeroFloatingResources := schedulerobjects.ResourceList{Resources: make(map[string]resource.Quantity, len(config))} for _, c := range config { if _, exists := zeroFloatingResources.Resources[c.Name]; exists { @@ -51,24 +53,21 @@ func NewFloatingResourceTypes(config []configuration.FloatingResourceConfig) (*F return &FloatingResourceTypes{ zeroFloatingResources: zeroFloatingResources, pools: pools, + rlFactory: rlFactory, }, nil } -func (frt *FloatingResourceTypes) WithinLimits(poolName string, allocated schedulerobjects.ResourceList) (bool, string) { - pool, exists := frt.pools[poolName] - if !exists { +func (frt *FloatingResourceTypes) WithinLimits(poolName string, allocated internaltypes.ResourceList) (bool, string) { + available := frt.GetTotalAvailableForPoolInternalTypes(poolName) + if available.AllZero() { return false, fmt.Sprintf("floating resources not connfigured for pool %s", poolName) } - rl := pool.totalResources.DeepCopy() - rl.Sub(allocated) - for resourceName, quantity := range rl.Resources { - if !frt.isFloatingResource(resourceName) { - continue - } - if quantity.Cmp(resource.Quantity{}) == -1 { - return false, fmt.Sprintf("not enough floating resource %s in pool %s", resourceName, poolName) - } + + resourceName, _, _, exceeds := allocated.OfType(internaltypes.Floating).ExceedsAvailable(available) + if exceeds { + return false, fmt.Sprintf("not enough floating resource %s in pool %s", resourceName, poolName) } + return true, "" } @@ -86,10 +85,8 @@ func (frt *FloatingResourceTypes) GetTotalAvailableForPool(poolName string) sche return pool.totalResources.DeepCopy() } -func (frt *FloatingResourceTypes) AddTotalAvailableForPool(poolName string, kubernetesResources schedulerobjects.ResourceList) schedulerobjects.ResourceList { - floatingResources := frt.GetTotalAvailableForPool(poolName) // Note GetTotalAvailableForPool returns a deep copy - floatingResources.Add(kubernetesResources) - return floatingResources +func (frt *FloatingResourceTypes) GetTotalAvailableForPoolInternalTypes(poolName string) internaltypes.ResourceList { + return frt.rlFactory.FromNodeProto(frt.GetTotalAvailableForPool(poolName).Resources) } func (frt *FloatingResourceTypes) SummaryString() string { @@ -98,8 +95,3 @@ func (frt *FloatingResourceTypes) SummaryString() string { } return strings.Join(maps.Keys(frt.zeroFloatingResources.Resources), " ") } - -func (frt *FloatingResourceTypes) isFloatingResource(resourceName string) bool { - _, exists := frt.zeroFloatingResources.Resources[resourceName] - return exists -} diff --git a/internal/scheduler/floatingresources/floating_resource_types_test.go b/internal/scheduler/floatingresources/floating_resource_types_test.go index 895510323b9..76ce183b624 100644 --- a/internal/scheduler/floatingresources/floating_resource_types_test.go +++ b/internal/scheduler/floatingresources/floating_resource_types_test.go @@ -7,82 +7,93 @@ import ( "k8s.io/apimachinery/pkg/api/resource" "github.com/armadaproject/armada/internal/scheduler/configuration" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" ) func TestAllPools(t *testing.T) { - sut := makeSut(t) + sut := makeSut(t, makeRlFactory()) assert.Equal(t, []string{"cpu", "gpu"}, sut.AllPools()) } func TestGetTotalAvailableForPool(t *testing.T) { - sut := makeSut(t) + sut := makeSut(t, makeRlFactory()) zero := resource.Quantity{} assert.Equal(t, map[string]resource.Quantity{"floating-resource-1": resource.MustParse("200"), "floating-resource-2": resource.MustParse("300")}, sut.GetTotalAvailableForPool("cpu").Resources) assert.Equal(t, map[string]resource.Quantity{"floating-resource-1": resource.MustParse("100"), "floating-resource-2": zero}, sut.GetTotalAvailableForPool("gpu").Resources) assert.Equal(t, map[string]resource.Quantity{"floating-resource-1": zero, "floating-resource-2": zero}, sut.GetTotalAvailableForPool("some-other-pool").Resources) } -func TestAddTotalAvailableForPool(t *testing.T) { - sut := makeSut(t) - zero := resource.Quantity{} - ten := *resource.NewQuantity(10, resource.DecimalSI) - kubernetesResources := schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"cpu": ten}} - assert.Equal(t, map[string]resource.Quantity{"cpu": ten, "floating-resource-1": resource.MustParse("200"), "floating-resource-2": resource.MustParse("300")}, sut.AddTotalAvailableForPool("cpu", kubernetesResources).Resources) - assert.Equal(t, map[string]resource.Quantity{"cpu": ten, "floating-resource-1": resource.MustParse("100"), "floating-resource-2": zero}, sut.AddTotalAvailableForPool("gpu", kubernetesResources).Resources) - assert.Equal(t, map[string]resource.Quantity{"cpu": ten, "floating-resource-1": zero, "floating-resource-2": zero}, sut.AddTotalAvailableForPool("some-other-pool", kubernetesResources).Resources) - assert.Equal(t, map[string]resource.Quantity{"cpu": ten}, kubernetesResources.Resources) // check hasn't mutated arg +func TestGetTotalAvailableForPoolInternalTypes(t *testing.T) { + sut := makeSut(t, makeRlFactory()) + + cpuPool := sut.GetTotalAvailableForPoolInternalTypes("cpu") + assert.Equal(t, int64(200000), cpuPool.GetByNameZeroIfMissing("floating-resource-1")) + assert.Equal(t, int64(300000), cpuPool.GetByNameZeroIfMissing("floating-resource-2")) + + gpuPool := sut.GetTotalAvailableForPoolInternalTypes("gpu") + assert.Equal(t, int64(100000), gpuPool.GetByNameZeroIfMissing("floating-resource-1")) + assert.Equal(t, int64(0), gpuPool.GetByNameZeroIfMissing("floating-resource-2")) + + notFound := sut.GetTotalAvailableForPoolInternalTypes("some-invalid-value") + assert.Equal(t, int64(0), notFound.GetByNameZeroIfMissing("floating-resource-1")) + assert.Equal(t, int64(0), notFound.GetByNameZeroIfMissing("floating-resource-2")) } func TestWithinLimits_WhenWithinLimits_ReturnsTrue(t *testing.T) { - sut := makeSut(t) + rlFactory := makeRlFactory() + sut := makeSut(t, rlFactory) withinLimits, errorMessage := sut.WithinLimits("cpu", - schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"floating-resource-1": resource.MustParse("199")}}, + rlFactory.FromJobResourceListIgnoreUnknown(map[string]resource.Quantity{"floating-resource-1": resource.MustParse("199")}), ) assert.True(t, withinLimits) assert.Empty(t, errorMessage) } func TestWithinLimits_WhenAtLimit_ReturnsTrue(t *testing.T) { - sut := makeSut(t) + rlFactory := makeRlFactory() + sut := makeSut(t, rlFactory) withinLimits, errorMessage := sut.WithinLimits("cpu", - schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"floating-resource-1": resource.MustParse("200")}}, + rlFactory.FromJobResourceListIgnoreUnknown(map[string]resource.Quantity{"floating-resource-1": resource.MustParse("200")}), ) assert.True(t, withinLimits) assert.Empty(t, errorMessage) } func TestWithinLimits_WhenExceedsLimit_ReturnsFalse(t *testing.T) { - sut := makeSut(t) + rlFactory := makeRlFactory() + sut := makeSut(t, rlFactory) withinLimits, errorMessage := sut.WithinLimits("cpu", - schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"floating-resource-1": resource.MustParse("201")}}, + rlFactory.FromJobResourceListIgnoreUnknown(map[string]resource.Quantity{"floating-resource-1": resource.MustParse("201")}), ) assert.False(t, withinLimits) assert.NotEmpty(t, errorMessage) } func TestWithinLimits_IgnoresNonFloatingResources(t *testing.T) { - sut := makeSut(t) + rlFactory := makeRlFactory() + sut := makeSut(t, rlFactory) withinLimits, errorMessage := sut.WithinLimits("cpu", - schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"some-other-resource": resource.MustParse("1000")}}, + rlFactory.FromJobResourceListIgnoreUnknown(map[string]resource.Quantity{"cpu": resource.MustParse("1000")}), ) assert.True(t, withinLimits) assert.Empty(t, errorMessage) } func TestWithinLimits_WhenResourceNotSpecifiedForAPool_ReturnsFalse(t *testing.T) { - sut := makeSut(t) + rlFactory := makeRlFactory() + sut := makeSut(t, rlFactory) withinLimits, errorMessage := sut.WithinLimits("gpu", - schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"floating-resource-2": resource.MustParse("1")}}, + rlFactory.FromJobResourceListIgnoreUnknown(map[string]resource.Quantity{"floating-resource-2": resource.MustParse("1")}), ) assert.False(t, withinLimits) assert.NotEmpty(t, errorMessage) } func TestWithinLimits_WhenPoolDoesNotExist_ReturnsFalse(t *testing.T) { - sut := makeSut(t) + rlFactory := makeRlFactory() + sut := makeSut(t, rlFactory) withinLimits, errorMessage := sut.WithinLimits("some-other-pool", - schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"floating-resource-1": resource.MustParse("1")}}, + rlFactory.FromJobResourceListIgnoreUnknown(map[string]resource.Quantity{"floating-resource-1": resource.MustParse("1")}), ) assert.False(t, withinLimits) assert.NotEmpty(t, errorMessage) @@ -115,8 +126,18 @@ func testConfig() []configuration.FloatingResourceConfig { } } -func makeSut(t *testing.T) *FloatingResourceTypes { - sut, err := NewFloatingResourceTypes(testConfig()) +func makeRlFactory() *internaltypes.ResourceListFactory { + rlFactory, err := internaltypes.NewResourceListFactory([]configuration.ResourceType{ + {Name: "cpu"}, + }, testConfig()) + if err != nil { + panic(err) + } + return rlFactory +} + +func makeSut(t *testing.T, rlFactory *internaltypes.ResourceListFactory) *FloatingResourceTypes { + sut, err := NewFloatingResourceTypes(testConfig(), rlFactory) assert.Nil(t, err) return sut } diff --git a/internal/scheduler/internaltypes/resource_fraction_list_test.go b/internal/scheduler/internaltypes/resource_fraction_list_test.go index a187fa778e1..b8224d62c77 100644 --- a/internal/scheduler/internaltypes/resource_fraction_list_test.go +++ b/internal/scheduler/internaltypes/resource_fraction_list_test.go @@ -19,7 +19,8 @@ func TestMax(t *testing.T) { factory := testFactory() assert.Equal(t, 0.0, testResourceFractionList(factory, -0.1, 0.0, 0.0).Max()) assert.Equal(t, 0.0, testResourceFractionList(factory, 0.0, 0.0, 0.0).Max()) - assert.Equal(t, 0.9, testResourceFractionList(factory, 0.1, 0.9, 0.7).Max()) + assert.Equal(t, 0.9, testResourceFractionList(factory, 0.2, 0.9, 0.1).Max()) + assert.Equal(t, 0.9, testResourceFractionList(factory, 0.9, 0.2, 0.1).Max()) } func TestMax_HandlesEmptyCorrectly(t *testing.T) { diff --git a/internal/scheduler/internaltypes/resource_list.go b/internal/scheduler/internaltypes/resource_list.go index be4b42825f2..dda39551103 100644 --- a/internal/scheduler/internaltypes/resource_list.go +++ b/internal/scheduler/internaltypes/resource_list.go @@ -2,6 +2,7 @@ package internaltypes import ( "fmt" + "math" "golang.org/x/exp/slices" k8sResource "k8s.io/apimachinery/pkg/api/resource" @@ -76,6 +77,19 @@ func (rl ResourceList) GetByNameZeroIfMissing(name string) int64 { return rl.resources[index] } +func (rl ResourceList) GetResourceByNameZeroIfMissing(name string) k8sResource.Quantity { + if rl.IsEmpty() { + return k8sResource.Quantity{} + } + + index, ok := rl.factory.nameToIndex[name] + if !ok { + return k8sResource.Quantity{} + } + + return *k8sResource.NewScaledQuantity(rl.resources[index], rl.factory.scales[index]) +} + func (rl ResourceList) GetResources() []Resource { if rl.IsEmpty() { return []Resource{} @@ -147,6 +161,15 @@ func (rl ResourceList) IsEmpty() bool { return rl.factory == nil } +func (rl ResourceList) Factory() *ResourceListFactory { + return rl.factory +} + +func (rl ResourceList) Exceeds(other ResourceList) bool { + _, _, _, exceeds := rl.ExceedsAvailable(other) + return exceeds +} + // ExceedsAvailable // - if any resource in this ResourceList is greater than the equivalent resource in param available, this function returns // - the name of the relevant resource @@ -195,6 +218,21 @@ func (rl ResourceList) OfType(t ResourceType) ResourceList { return ResourceList{factory: rl.factory, resources: result} } +func (rl ResourceList) Cap(cap ResourceList) ResourceList { + assertSameResourceListFactory(rl.factory, cap.factory) + if rl.IsEmpty() { + return ResourceList{} + } + if cap.IsEmpty() { + return rl + } + result := make([]int64, len(rl.resources)) + for i, r := range rl.resources { + result[i] = min(r, cap.resources[i]) + } + return ResourceList{factory: rl.factory, resources: result} +} + func (rl ResourceList) Add(other ResourceList) ResourceList { assertSameResourceListFactory(rl.factory, other.factory) if rl.IsEmpty() { @@ -266,17 +304,6 @@ func (rl ResourceList) Negate() ResourceList { return ResourceList{factory: rl.factory, resources: result} } -func (rl ResourceList) Scale(factor float64) ResourceList { - if rl.IsEmpty() { - return rl - } - result := make([]int64, len(rl.resources)) - for i, r := range rl.resources { - result[i] = multiplyResource(r, factor) - } - return ResourceList{resources: result, factory: rl.factory} -} - func (rl ResourceList) asQuantity(index int) *k8sResource.Quantity { if rl.factory == nil { return &k8sResource.Quantity{} @@ -299,8 +326,21 @@ func assertSameResourceListFactory(a, b *ResourceListFactory) { func multiplyResource(res int64, multiplier float64) int64 { if multiplier == 1.0 { - // avoid rounding error in the simple case + // Avoid rounding error in the simple case. return res } + + // Return max int64 if multiplier is infinity. + // If res is zero, we assume infinity trumps zero, and return int64 maxValue. + // This gives the right behavior when the result is used as a cap, + // as an infinity multiplier means "never apply cap". + if math.IsInf(multiplier, 0) { + if (multiplier < 0) == (res < 0) { + return math.MaxInt64 + } else { + return math.MinInt64 + } + } + return int64(float64(res) * multiplier) } diff --git a/internal/scheduler/internaltypes/resource_list_map_util.go b/internal/scheduler/internaltypes/resource_list_map_util.go new file mode 100644 index 00000000000..b359703034a --- /dev/null +++ b/internal/scheduler/internaltypes/resource_list_map_util.go @@ -0,0 +1,59 @@ +package internaltypes + +import ( + "strings" + + "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" +) + +func RlMapToString(m map[string]ResourceList) string { + results := []string{} + for k, v := range m { + results = append(results, k+"="+v.String()) + } + return strings.Join(results, " ") +} + +func RlMapSumValues(m map[string]ResourceList) ResourceList { + result := ResourceList{} + for _, v := range m { + result = result.Add(v) + } + return result +} + +func RlMapAllZero(m map[string]ResourceList) bool { + for _, v := range m { + if !v.AllZero() { + return false + } + } + return true +} + +func RlMapHasNegativeValues(m map[string]ResourceList) bool { + for _, v := range m { + if v.HasNegativeValues() { + return true + } + } + return false +} + +func RlMapFromJobSchedulerObjects(m schedulerobjects.QuantityByTAndResourceType[string], rlFactory *ResourceListFactory) map[string]ResourceList { + result := map[string]ResourceList{} + for k, v := range m { + result[k] = rlFactory.FromJobResourceListIgnoreUnknown(v.Resources) + } + return result +} + +func RlMapRemoveZeros(m map[string]ResourceList) map[string]ResourceList { + result := map[string]ResourceList{} + for k, v := range m { + if !v.AllZero() { + result[k] = v + } + } + return result +} diff --git a/internal/scheduler/internaltypes/resource_list_map_util_test.go b/internal/scheduler/internaltypes/resource_list_map_util_test.go new file mode 100644 index 00000000000..f278eeefd72 --- /dev/null +++ b/internal/scheduler/internaltypes/resource_list_map_util_test.go @@ -0,0 +1,111 @@ +package internaltypes + +import ( + "testing" + + "github.com/stretchr/testify/assert" + "k8s.io/apimachinery/pkg/api/resource" + + "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" +) + +func TestRlMapSumValues(t *testing.T) { + factory := testFactory() + + assert.Equal(t, testResourceList(factory, "3", "3Ki"), RlMapSumValues(testMapAllPositive(factory))) + assert.True(t, RlMapSumValues(testMapEmpty(factory)).IsEmpty()) +} + +func TestRlMapAllZero(t *testing.T) { + factory := testFactory() + + assert.False(t, RlMapAllZero(testMapAllPositive(factory))) + assert.True(t, RlMapAllZero(testMapAllZero(factory))) + assert.False(t, RlMapAllZero(testMapOneZero(factory))) + assert.False(t, RlMapAllZero(testMapOneNegative(factory))) + assert.True(t, RlMapAllZero(testMapEmpty(factory))) +} + +func TestRlMapHasNegativeValues(t *testing.T) { + factory := testFactory() + + assert.False(t, RlMapHasNegativeValues(testMapAllPositive(factory))) + assert.False(t, RlMapHasNegativeValues(testMapAllZero(factory))) + assert.False(t, RlMapHasNegativeValues(testMapOneZero(factory))) + assert.True(t, RlMapHasNegativeValues(testMapOneNegative(factory))) + assert.False(t, RlMapHasNegativeValues(testMapEmpty(factory))) +} + +func TestRlMapFromJobSchedulerObjects(t *testing.T) { + factory := testFactory() + + input := make(schedulerobjects.QuantityByTAndResourceType[string]) + input.AddResourceList("priorityClass1", + schedulerobjects.ResourceList{ + Resources: map[string]resource.Quantity{ + "cpu": resource.MustParse("1"), + "memory": resource.MustParse("1Ki"), + }, + }, + ) + input.AddResourceList("priorityClass2", + schedulerobjects.ResourceList{ + Resources: map[string]resource.Quantity{ + "cpu": resource.MustParse("2"), + "memory": resource.MustParse("2Ki"), + }, + }, + ) + + expected := map[string]ResourceList{ + "priorityClass1": testResourceList(factory, "1", "1Ki"), + "priorityClass2": testResourceList(factory, "2", "2Ki"), + } + + assert.Equal(t, expected, RlMapFromJobSchedulerObjects(input, factory)) +} + +func TestRlMapRemoveZeros(t *testing.T) { + factory := testFactory() + + input := map[string]ResourceList{ + "priorityClass1": testResourceList(factory, "1", "0Ki"), + "priorityClass2": testResourceList(factory, "0", "0Ki"), + } + expected := map[string]ResourceList{ + "priorityClass1": testResourceList(factory, "1", "0Ki"), + } + assert.Equal(t, expected, RlMapRemoveZeros(input)) +} + +func testMapAllPositive(factory *ResourceListFactory) map[string]ResourceList { + return map[string]ResourceList{ + "a": testResourceList(factory, "1", "1Ki"), + "b": testResourceList(factory, "2", "2Ki"), + } +} + +func testMapAllZero(factory *ResourceListFactory) map[string]ResourceList { + return map[string]ResourceList{ + "a": testResourceList(factory, "0", "0"), + "b": testResourceList(factory, "0", "0"), + } +} + +func testMapOneNegative(factory *ResourceListFactory) map[string]ResourceList { + return map[string]ResourceList{ + "a": testResourceList(factory, "-1", "1Ki"), + "b": testResourceList(factory, "2", "2Ki"), + } +} + +func testMapOneZero(factory *ResourceListFactory) map[string]ResourceList { + return map[string]ResourceList{ + "a": testResourceList(factory, "0", "1Ki"), + "b": testResourceList(factory, "2", "2Ki"), + } +} + +func testMapEmpty(factory *ResourceListFactory) map[string]ResourceList { + return map[string]ResourceList{} +} diff --git a/internal/scheduler/internaltypes/resource_list_test.go b/internal/scheduler/internaltypes/resource_list_test.go index 216ab626670..1138b6225b2 100644 --- a/internal/scheduler/internaltypes/resource_list_test.go +++ b/internal/scheduler/internaltypes/resource_list_test.go @@ -1,6 +1,7 @@ package internaltypes import ( + "math" "testing" "github.com/stretchr/testify/assert" @@ -64,6 +65,19 @@ func TestGetByNameZeroIfMissing_HandlesEmptyCorrectly(t *testing.T) { assert.Equal(t, int64(0), empty.GetByNameZeroIfMissing("missing")) } +func TestGetResourceByNameZeroIfMissing(t *testing.T) { + factory := testFactory() + a := testResourceList(factory, "1", "1Gi") + + assert.Equal(t, *k8sResource.NewScaledQuantity(1000, k8sResource.Milli), a.GetResourceByNameZeroIfMissing("cpu")) + assert.Equal(t, k8sResource.Quantity{}, a.GetResourceByNameZeroIfMissing("missing")) +} + +func TestGetResourceByNameZeroIfMissing_HandlesEmptyCorrectly(t *testing.T) { + empty := ResourceList{} + assert.Equal(t, k8sResource.Quantity{}, empty.GetResourceByNameZeroIfMissing("missing")) +} + func TestGetResources(t *testing.T) { factory := testFactory() a := testResourceList(factory, "1", "1Gi") @@ -222,6 +236,38 @@ func TestOfType_HandlesEmptyCorrectly(t *testing.T) { assert.Equal(t, ResourceList{}, ResourceList{}.OfType(Floating)) } +func TestCap(t *testing.T) { + factory := testFactory() + + assert.Equal(t, testResourceList(factory, "1", "2Ki"), testResourceList(factory, "1", "2Ki").Cap(testResourceList(factory, "2", "4Ki"))) + assert.Equal(t, testResourceList(factory, "1", "1Ki"), testResourceList(factory, "1", "4Ki").Cap(testResourceList(factory, "2", "1Ki"))) +} + +func TestCap_HandlesEmptyCorrectly(t *testing.T) { + factory := testFactory() + + assert.Equal(t, testResourceList(factory, "1", "1Ki"), testResourceList(factory, "1", "1Ki").Cap(ResourceList{})) + assert.Equal(t, ResourceList{}, ResourceList{}.Cap(testResourceList(factory, "1", "1Ki"))) + assert.Equal(t, ResourceList{}, ResourceList{}.Cap(ResourceList{})) +} + +func TestExceeds(t *testing.T) { + factory := testFactory() + + assert.False(t, testResourceList(factory, "0", "0Ki").Exceeds(testResourceList(factory, "0", "0Ki"))) + assert.False(t, testResourceList(factory, "1", "1Ki").Exceeds(testResourceList(factory, "1", "1Ki"))) + assert.True(t, testResourceList(factory, "2", "1Ki").Exceeds(testResourceList(factory, "1", "1Ki"))) + assert.False(t, testResourceList(factory, "1", "1Ki").Exceeds(testResourceList(factory, "2", "1Ki"))) +} + +func TestExceeds_HandlesEmptyCorrectly(t *testing.T) { + factory := testFactory() + + assert.True(t, testResourceList(factory, "1", "1Ki").Exceeds(ResourceList{})) + assert.False(t, ResourceList{}.Exceeds(testResourceList(factory, "1", "1Ki"))) + assert.False(t, ResourceList{}.Exceeds(ResourceList{})) +} + func TestAdd(t *testing.T) { factory := testFactory() @@ -273,6 +319,34 @@ func TestMultiply(t *testing.T) { testResourceFractionList(factory, -0.25, -0.75, 1))) } +func TestMultiply_HandlesInfinityCorrectly(t *testing.T) { + factory := testFactory() + + result1 := testResourceList(factory, "100", "100Ki").Multiply(testResourceFractionList(factory, 0.75, math.Inf(1), 1)) + assert.Equal(t, int64(75000), result1.GetByNameZeroIfMissing("cpu")) + assert.Equal(t, int64(math.MaxInt64), result1.GetByNameZeroIfMissing("memory")) + + result2 := testResourceList(factory, "100", "0").Multiply(testResourceFractionList(factory, 0.75, math.Inf(1), 1)) + assert.Equal(t, int64(math.MaxInt64), result2.GetByNameZeroIfMissing("memory")) + + result3 := testResourceList(factory, "100", "-100Ki").Multiply(testResourceFractionList(factory, 0.75, math.Inf(1), 1)) + assert.Equal(t, int64(math.MinInt64), result3.GetByNameZeroIfMissing("memory")) +} + +func TestMultiply_HandlesMinusInfinityCorrectly(t *testing.T) { + factory := testFactory() + + result1 := testResourceList(factory, "100", "100Ki").Multiply(testResourceFractionList(factory, 0.75, math.Inf(-1), 1)) + assert.Equal(t, int64(75000), result1.GetByNameZeroIfMissing("cpu")) + assert.Equal(t, int64(math.MinInt64), result1.GetByNameZeroIfMissing("memory")) + + result2 := testResourceList(factory, "100", "0").Multiply(testResourceFractionList(factory, 0.75, math.Inf(-1), 1)) + assert.Equal(t, int64(math.MinInt64), result2.GetByNameZeroIfMissing("memory")) + + result3 := testResourceList(factory, "100", "-100Ki").Multiply(testResourceFractionList(factory, 0.75, math.Inf(-1), 1)) + assert.Equal(t, int64(math.MaxInt64), result3.GetByNameZeroIfMissing("memory")) +} + func TestMultiply_HandlesEmptyCorrectly(t *testing.T) { factory := testFactory() @@ -316,19 +390,6 @@ func TestNegate_HandlesEmptyCorrectly(t *testing.T) { assert.Equal(t, ResourceList{}, ResourceList{}.Negate()) } -func TestScale(t *testing.T) { - factory := testFactory() - assert.Equal(t, testResourceList(factory, "4", "2Ki"), testResourceList(factory, "2", "1Ki").Scale(2.0)) - assert.Equal(t, testResourceList(factory, "2", "1Ki"), testResourceList(factory, "2", "1Ki").Scale(1.0)) - assert.Equal(t, testResourceList(factory, "0", "0Ki"), testResourceList(factory, "2", "1Ki").Scale(0.0)) - assert.Equal(t, testResourceList(factory, "2", "-1Ki"), testResourceList(factory, "-2", "1Ki").Scale(-1.0)) -} - -func TestScale_HandlesEmptyCorrectly(t *testing.T) { - assert.Equal(t, ResourceList{}, ResourceList{}.Scale(0.0)) - assert.Equal(t, ResourceList{}, ResourceList{}.Scale(1.0)) -} - func testResourceList(factory *ResourceListFactory, cpu string, memory string) ResourceList { return factory.FromJobResourceListIgnoreUnknown(map[string]k8sResource.Quantity{ "cpu": k8sResource.MustParse(cpu), diff --git a/internal/scheduler/metrics/cycle_metrics_test.go b/internal/scheduler/metrics/cycle_metrics_test.go index 86c810b6fa2..4c07b001413 100644 --- a/internal/scheduler/metrics/cycle_metrics_test.go +++ b/internal/scheduler/metrics/cycle_metrics_test.go @@ -11,7 +11,7 @@ import ( "k8s.io/apimachinery/pkg/api/resource" "github.com/armadaproject/armada/internal/scheduler/configuration" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/scheduling" "github.com/armadaproject/armada/internal/scheduler/scheduling/context" "github.com/armadaproject/armada/internal/scheduler/scheduling/fairness" @@ -23,7 +23,8 @@ const epsilon = 1e-6 func TestReportStateTransitions(t *testing.T) { fairnessCostProvider, err := fairness.NewDominantResourceFairness( cpu(100), - configuration.SchedulingConfig{DominantResourceFairnessResourcesToConsider: []string{"cpu"}}) + configuration.SchedulingConfig{DominantResourceFairnessResourcesToConsider: []string{"cpu"}}, + ) require.NoError(t, err) result := scheduling.SchedulerResult{ SchedulingContexts: []*context.SchedulingContext{ @@ -171,8 +172,8 @@ func TestDisableLeaderMetrics(t *testing.T) { assert.NotZero(t, len(collect(m))) } -func cpu(n int) schedulerobjects.ResourceList { - return schedulerobjects.ResourceList{ - Resources: map[string]resource.Quantity{"cpu": resource.MustParse(fmt.Sprintf("%d", n))}, - } +func cpu(n int) internaltypes.ResourceList { + return testfixtures.TestResourceListFactory.FromJobResourceListIgnoreUnknown( + map[string]resource.Quantity{"cpu": resource.MustParse(fmt.Sprintf("%d", n))}, + ) } diff --git a/internal/scheduler/nodedb/nodedb.go b/internal/scheduler/nodedb/nodedb.go index 7d257de5a4e..336ca0de314 100644 --- a/internal/scheduler/nodedb/nodedb.go +++ b/internal/scheduler/nodedb/nodedb.go @@ -22,7 +22,6 @@ import ( "github.com/armadaproject/armada/internal/scheduler/configuration" "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" "github.com/armadaproject/armada/internal/scheduler/scheduling/context" ) @@ -213,7 +212,7 @@ func NewNodeDb( nodeTypes: make(map[uint64]*internaltypes.NodeType), wellKnownNodeTypes: make(map[string]*configuration.WellKnownNodeType), numNodesByNodeType: make(map[uint64]int), - totalResources: internaltypes.ResourceList{}, + totalResources: resourceListFactory.MakeAllZero(), db: db, // Set the initial capacity (somewhat arbitrarily) to 128 reasons. podRequirementsNotMetReasonStringCache: make(map[uint64]string, 128), @@ -305,10 +304,10 @@ func (nodeDb *NodeDb) NumNodes() int { return int(nodeDb.numNodes) } -func (nodeDb *NodeDb) TotalKubernetesResources() schedulerobjects.ResourceList { +func (nodeDb *NodeDb) TotalKubernetesResources() internaltypes.ResourceList { nodeDb.mu.Lock() defer nodeDb.mu.Unlock() - return schedulerobjects.ResourceList{Resources: nodeDb.totalResources.ToMap()} + return nodeDb.totalResources } func (nodeDb *NodeDb) Txn(write bool) *memdb.Txn { diff --git a/internal/scheduler/nodedb/nodedb_test.go b/internal/scheduler/nodedb/nodedb_test.go index fd1a53b6281..1f23597541b 100644 --- a/internal/scheduler/nodedb/nodedb_test.go +++ b/internal/scheduler/nodedb/nodedb_test.go @@ -31,9 +31,10 @@ func TestTotalResources(t *testing.T) { nodeDb, err := newNodeDbWithNodes([]*schedulerobjects.Node{}) require.NoError(t, err) - expected := schedulerobjects.ResourceList{Resources: make(map[string]resource.Quantity)} - assert.True(t, expected.Equal(nodeDb.TotalKubernetesResources())) + assert.False(t, nodeDb.TotalKubernetesResources().IsEmpty()) + assert.True(t, nodeDb.TotalKubernetesResources().AllZero()) + expected := schedulerobjects.ResourceList{Resources: make(map[string]resource.Quantity)} // Upserting nodes for the first time should increase the resource count. nodes := testfixtures.N32CpuNodes(2, testfixtures.TestPriorities) for _, node := range nodes { @@ -48,7 +49,9 @@ func TestTotalResources(t *testing.T) { } txn.Commit() - assert.True(t, expected.Equal(nodeDb.TotalKubernetesResources())) + assert.True(t, expected.Equal(schedulerobjects.ResourceList{ + Resources: nodeDb.TotalKubernetesResources().ToMap(), + })) // Upserting new nodes should increase the resource count. nodes = testfixtures.N8GpuNodes(3, testfixtures.TestPriorities) @@ -64,7 +67,9 @@ func TestTotalResources(t *testing.T) { } txn.Commit() - assert.True(t, expected.Equal(nodeDb.TotalKubernetesResources())) + assert.True(t, expected.Equal(schedulerobjects.ResourceList{ + Resources: nodeDb.TotalKubernetesResources().ToMap(), + })) } func TestSelectNodeForPod_NodeIdLabel_Success(t *testing.T) { diff --git a/internal/scheduler/schedulerapp.go b/internal/scheduler/schedulerapp.go index 0ed82388e46..902d3adca25 100644 --- a/internal/scheduler/schedulerapp.go +++ b/internal/scheduler/schedulerapp.go @@ -85,7 +85,7 @@ func Run(config schedulerconfig.Configuration) error { } ctx.Infof("Supported resource types: %s", resourceListFactory.SummaryString()) - floatingResourceTypes, err := floatingresources.NewFloatingResourceTypes(config.Scheduling.ExperimentalFloatingResources) + floatingResourceTypes, err := floatingresources.NewFloatingResourceTypes(config.Scheduling.ExperimentalFloatingResources, resourceListFactory) if err != nil { return err } diff --git a/internal/scheduler/scheduling/constraints/constraints.go b/internal/scheduler/scheduling/constraints/constraints.go index 1f8a5b4faf3..270dd8a2dfb 100644 --- a/internal/scheduler/scheduling/constraints/constraints.go +++ b/internal/scheduler/scheduling/constraints/constraints.go @@ -4,15 +4,23 @@ import ( "math" "github.com/pkg/errors" - "k8s.io/apimachinery/pkg/api/resource" + armadamaps "github.com/armadaproject/armada/internal/common/maps" + "github.com/armadaproject/armada/internal/common/types" "github.com/armadaproject/armada/internal/common/util" "github.com/armadaproject/armada/internal/scheduler/configuration" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/scheduling/context" "github.com/armadaproject/armada/pkg/api" ) +// SchedulingConstraints contains scheduling constraints, e.g. per-queue resource limits. +type SchedulingConstraints interface { + CheckRoundConstraints(sctx *context.SchedulingContext) (bool, string, error) + CheckJobConstraints(sctx *context.SchedulingContext, gctx *context.GangSchedulingContext) (bool, string, error) + CapResources(queue string, resourcesByPc map[string]internaltypes.ResourceList) map[string]internaltypes.ResourceList +} + const ( // Indicates that the limit on resources scheduled per round has been exceeded. MaximumResourcesScheduledUnschedulableReason = "maximum resources scheduled" @@ -23,7 +31,7 @@ const ( // Indicates that the scheduling rate limit has been exceeded. GlobalRateLimitExceededUnschedulableReason = "global scheduling rate limit exceeded" QueueRateLimitExceededUnschedulableReason = "queue scheduling rate limit exceeded" - SchedulingPausedOnQueueUnschedulableReason = "scheduling paused on queue" + QueueCordonedUnschedulableReason = "queue cordoned" // Indicates that scheduling a gang would exceed the rate limit. GlobalRateLimitExceededByGangUnschedulableReason = "gang would exceed global scheduling rate limit" @@ -55,107 +63,45 @@ func IsTerminalUnschedulableReason(reason string) bool { // IsTerminalQueueUnschedulableReason returns true if reason indicates // it's not possible to schedule any more jobs from this queue in this round. func IsTerminalQueueUnschedulableReason(reason string) bool { - return reason == QueueRateLimitExceededUnschedulableReason || reason == SchedulingPausedOnQueueUnschedulableReason + return reason == QueueRateLimitExceededUnschedulableReason || reason == QueueCordonedUnschedulableReason } // SchedulingConstraints contains scheduling constraints, e.g., per-queue resource limits. -type SchedulingConstraints struct { - // Scheduling constraints by priority class. - priorityClassSchedulingConstraintsByPriorityClassName map[string]priorityClassSchedulingConstraints - // Scheduling constraints for specific queues. - // If present for a particular queue, global limits (i.e., priorityClassSchedulingConstraintsByPriorityClassName) - // do not apply for that queue. - queueSchedulingConstraintsByQueueName map[string]queueSchedulingConstraints - // Limits total resources scheduled per invocation. - maximumResourcesToSchedule map[string]resource.Quantity -} - -// queueSchedulingConstraints contains per-queue scheduling constraints. -type queueSchedulingConstraints struct { - // Scheduling constraints by priority class. - PriorityClassSchedulingConstraintsByPriorityClassName map[string]priorityClassSchedulingConstraints - // Determines whether scheduling has been paused for this queue - Cordoned bool -} - -// priorityClassSchedulingConstraints contains scheduling constraints that apply to jobs of a specific priority class. -type priorityClassSchedulingConstraints struct { - PriorityClassName string - // Limits total resources allocated to jobs of this priority class per queue. - MaximumResourcesPerQueue map[string]resource.Quantity +type schedulingConstraints struct { + // Limits total resources scheduled per scheduling round. + maximumResourcesToSchedule internaltypes.ResourceList + // Queues that are cordoned (i.e. no jobs may be scheduled on them) + cordonedQueues map[string]bool + // Resource limits by queue and priority class. E.g. "Queue A is limited to 100 cpu at priority class armada-default" + resourceLimitsPerQueuePerPriorityClass map[string]map[string]internaltypes.ResourceList } -func NewSchedulingConstraints(pool string, totalResources schedulerobjects.ResourceList, config configuration.SchedulingConfig, queues []*api.Queue) SchedulingConstraints { - priorityClassSchedulingConstraintsByPriorityClassName := make(map[string]priorityClassSchedulingConstraints, len(config.PriorityClasses)) - for name, priorityClass := range config.PriorityClasses { - maximumResourceFractionPerQueue := priorityClass.MaximumResourceFractionPerQueue - if m, ok := priorityClass.MaximumResourceFractionPerQueueByPool[pool]; ok { - // Use pool-specific config is available. - maximumResourceFractionPerQueue = util.MergeMaps(maximumResourceFractionPerQueue, m) - } - priorityClassSchedulingConstraintsByPriorityClassName[name] = priorityClassSchedulingConstraints{ - PriorityClassName: name, - MaximumResourcesPerQueue: absoluteFromRelativeLimits(totalResources.Resources, maximumResourceFractionPerQueue), - } - } - - queueSchedulingConstraintsByQueueName := make(map[string]queueSchedulingConstraints, len(queues)) - for _, queue := range queues { - priorityClassSchedulingConstraintsByPriorityClassNameForQueue := make(map[string]priorityClassSchedulingConstraints, len(queue.ResourceLimitsByPriorityClassName)) - for priorityClassName, priorityClassResourceLimits := range queue.ResourceLimitsByPriorityClassName { - maximumResourceFraction := priorityClassResourceLimits.MaximumResourceFraction - if m, ok := priorityClassResourceLimits.MaximumResourceFractionByPool[pool]; ok { - // Use pool-specific maximum resource fraction if available. - maximumResourceFraction = util.MergeMaps(maximumResourceFraction, m.MaximumResourceFraction) - } - priorityClassSchedulingConstraintsByPriorityClassNameForQueue[priorityClassName] = priorityClassSchedulingConstraints{ - PriorityClassName: priorityClassName, - MaximumResourcesPerQueue: absoluteFromRelativeLimits(totalResources.Resources, maximumResourceFraction), - } - } - queueSchedulingConstraintsByQueueName[queue.Name] = queueSchedulingConstraints{ - PriorityClassSchedulingConstraintsByPriorityClassName: priorityClassSchedulingConstraintsByPriorityClassNameForQueue, - Cordoned: queue.Cordoned, - } - } +func NewSchedulingConstraints( + pool string, + totalResources internaltypes.ResourceList, + config configuration.SchedulingConfig, + queues []*api.Queue, +) SchedulingConstraints { + cordonedQueues := armadamaps.FromSlice(queues, + func(q *api.Queue) string { return q.Name }, + func(q *api.Queue) bool { return q.Cordoned }) - maximumResourceFractionToSchedule := config.MaximumResourceFractionToSchedule - if m, ok := config.MaximumResourceFractionToScheduleByPool[pool]; ok { - // Use pool-specific config is available. - maximumResourceFractionToSchedule = m - } - return SchedulingConstraints{ - maximumResourcesToSchedule: absoluteFromRelativeLimits(totalResources.Resources, maximumResourceFractionToSchedule), - priorityClassSchedulingConstraintsByPriorityClassName: priorityClassSchedulingConstraintsByPriorityClassName, - queueSchedulingConstraintsByQueueName: queueSchedulingConstraintsByQueueName, + return &schedulingConstraints{ + cordonedQueues: cordonedQueues, + maximumResourcesToSchedule: calculatePerRoundLimits(totalResources, pool, config), + resourceLimitsPerQueuePerPriorityClass: calculatePerQueueLimits(totalResources, pool, config.PriorityClasses, queues), } } -func absoluteFromRelativeLimits(totalResources map[string]resource.Quantity, relativeLimits map[string]float64) map[string]resource.Quantity { - absoluteLimits := make(map[string]resource.Quantity, len(relativeLimits)) - for t, f := range relativeLimits { - absoluteLimits[t] = ScaleQuantity(totalResources[t].DeepCopy(), f) - } - return absoluteLimits -} - -// ScaleQuantity scales q in-place by a factor f. -// This functions overflows for quantities the milli value of which can't be expressed as an int64. -// E.g., 1Pi is ok, but not 10Pi. -func ScaleQuantity(q resource.Quantity, f float64) resource.Quantity { - q.SetMilli(int64(math.Round(float64(q.MilliValue()) * f))) - return q -} - -func (constraints *SchedulingConstraints) CheckRoundConstraints(sctx *context.SchedulingContext) (bool, string, error) { +func (constraints *schedulingConstraints) CheckRoundConstraints(sctx *context.SchedulingContext) (bool, string, error) { // maximumResourcesToSchedule check. - if !isStrictlyLessOrEqual(sctx.ScheduledResources.Resources, constraints.maximumResourcesToSchedule) { + if sctx.ScheduledResources.Exceeds(constraints.maximumResourcesToSchedule) { return false, MaximumResourcesScheduledUnschedulableReason, nil } return true, "", nil } -func (constraints *SchedulingConstraints) CheckConstraints( +func (constraints *schedulingConstraints) CheckJobConstraints( sctx *context.SchedulingContext, gctx *context.GangSchedulingContext, ) (bool, string, error) { @@ -164,6 +110,11 @@ func (constraints *SchedulingConstraints) CheckConstraints( return false, "", errors.Errorf("no QueueSchedulingContext for queue %s", gctx.Queue) } + // Queue cordoned + if constraints.cordonedQueues[qctx.Queue] { + return false, QueueCordonedUnschedulableReason, nil + } + // Global rate limiter check. tokens := sctx.Limiter.TokensAt(sctx.Started) if tokens <= 0 { @@ -176,9 +127,6 @@ func (constraints *SchedulingConstraints) CheckConstraints( return false, GlobalRateLimitExceededByGangUnschedulableReason, nil } - if queueConstraints, ok := constraints.queueSchedulingConstraintsByQueueName[qctx.Queue]; ok && queueConstraints.Cordoned { - return false, SchedulingPausedOnQueueUnschedulableReason, nil - } // Per-queue rate limiter check. tokens = qctx.Limiter.TokensAt(sctx.Started) if tokens <= 0 { @@ -191,64 +139,83 @@ func (constraints *SchedulingConstraints) CheckConstraints( return false, QueueRateLimitExceededByGangUnschedulableReason, nil } - // queueSchedulingConstraintsByQueueName / priorityClassSchedulingConstraintsByPriorityClassName checks. - overallResourceLimits := constraints.resolveResourceLimitsForQueueAndPriorityClass(gctx.Queue, gctx.PriorityClassName) - if !isStrictlyLessOrEqual(qctx.AllocatedByPriorityClass[gctx.PriorityClassName].Resources, overallResourceLimits) { + // Quantity scheduled by queue and priority class + queueLimit, haslimit := constraints.resourceLimitsPerQueuePerPriorityClass[qctx.Queue][gctx.PriorityClassName] + allocatedResources := qctx.AllocatedByPriorityClass[gctx.PriorityClassName] + if haslimit && allocatedResources.Exceeds(queueLimit) { return false, UnschedulableReasonMaximumResourcesExceeded, nil } return true, "", nil } -func (constraints *SchedulingConstraints) CapResources(queue string, resourcesByPc schedulerobjects.QuantityByTAndResourceType[string]) schedulerobjects.QuantityByTAndResourceType[string] { - cappedResourcesByPc := schedulerobjects.QuantityByTAndResourceType[string]{} +func (c *schedulingConstraints) CapResources(queue string, resourcesByPc map[string]internaltypes.ResourceList) map[string]internaltypes.ResourceList { + perQueueLimit, ok := c.resourceLimitsPerQueuePerPriorityClass[queue] + if !ok { + return resourcesByPc + } + cappedResourcesByPc := make(map[string]internaltypes.ResourceList, len(resourcesByPc)) for pc, resources := range resourcesByPc { - overallResourceLimits := constraints.resolveResourceLimitsForQueueAndPriorityClass(queue, pc) - cappedResources := make(map[string]resource.Quantity, len(resources.Resources)) - for resourceName, qty := range resources.Resources { - limit, ok := overallResourceLimits[resourceName] - if ok && qty.Cmp(limit) == 1 { - cappedResources[resourceName] = limit - } else { - cappedResources[resourceName] = qty - } - } - cappedResourcesByPc[pc] = schedulerobjects.ResourceList{Resources: cappedResources} + cappedResourcesByPc[pc] = resources.Cap(perQueueLimit[pc]) } return cappedResourcesByPc } -func (constraints *SchedulingConstraints) resolveResourceLimitsForQueueAndPriorityClass(queue string, priorityClass string) map[string]resource.Quantity { - queueAndPriorityClassResourceLimits := constraints.getQueueAndPriorityClassResourceLimits(queue, priorityClass) - priorityClassResourceLimits := constraints.getPriorityClassResourceLimits(priorityClass) - return util.MergeMaps(priorityClassResourceLimits, queueAndPriorityClassResourceLimits) -} - -func (constraints *SchedulingConstraints) getQueueAndPriorityClassResourceLimits(queue string, priorityClass string) map[string]resource.Quantity { - if queueConstraint, ok := constraints.queueSchedulingConstraintsByQueueName[queue]; ok { - if priorityClassConstraint, ok := queueConstraint.PriorityClassSchedulingConstraintsByPriorityClassName[priorityClass]; ok { - return priorityClassConstraint.MaximumResourcesPerQueue - } +func calculatePerRoundLimits( + totalResources internaltypes.ResourceList, + pool string, + config configuration.SchedulingConfig, +) internaltypes.ResourceList { + if totalResources.IsEmpty() { + return totalResources } - return map[string]resource.Quantity{} -} + rlFactory := totalResources.Factory() -func (constraints *SchedulingConstraints) getPriorityClassResourceLimits(priorityClass string) map[string]resource.Quantity { - if priorityClassConstraint, ok := constraints.priorityClassSchedulingConstraintsByPriorityClassName[priorityClass]; ok { - return priorityClassConstraint.MaximumResourcesPerQueue + maximumResourceFractionToSchedule := config.MaximumResourceFractionToSchedule + if m, ok := config.MaximumResourceFractionToScheduleByPool[pool]; ok { + // Use pool-specific config is available. + // Should do util.MergeMaps really but don't want to change existing behaviour. + maximumResourceFractionToSchedule = m } - return map[string]resource.Quantity{} -} + return totalResources.Multiply(rlFactory.MakeResourceFractionList(maximumResourceFractionToSchedule, math.Inf(1))) +} + +func calculatePerQueueLimits( + totalResources internaltypes.ResourceList, + pool string, + priorityClasses map[string]types.PriorityClass, + queues []*api.Queue, +) map[string]map[string]internaltypes.ResourceList { + limitsPerQueuePerPc := make(map[string]map[string]internaltypes.ResourceList, len(queues)) + + if totalResources.IsEmpty() { + return limitsPerQueuePerPc + } + rlFactory := totalResources.Factory() + + for pcName, pc := range priorityClasses { + defaultFractions := util.MergeMaps( + pc.MaximumResourceFractionPerQueue, + pc.MaximumResourceFractionPerQueueByPool[pool], + ) + + for _, queue := range queues { + fractions := defaultFractions + queueConfig, ok := queue.ResourceLimitsByPriorityClassName[pcName] + if ok { + fractions = util.MergeMaps(fractions, queueConfig.MaximumResourceFraction) + queuePoolConfig, ok := queueConfig.MaximumResourceFractionByPool[pool] + if ok { + fractions = util.MergeMaps(fractions, queuePoolConfig.GetMaximumResourceFraction()) + } + } -// isStrictlyLessOrEqual returns false if -// - there is a quantity in b greater than that in a or -// - there is a non-zero quantity in b not in a -// and true otherwise. -func isStrictlyLessOrEqual(a map[string]resource.Quantity, b map[string]resource.Quantity) bool { - for t, q := range b { - if q.Cmp(a[t]) == -1 { - return false + if _, ok := limitsPerQueuePerPc[queue.Name]; !ok { + limitsPerQueuePerPc[queue.Name] = map[string]internaltypes.ResourceList{} + } + limitsPerQueuePerPc[queue.Name][pcName] = totalResources.Multiply(rlFactory.MakeResourceFractionList(fractions, math.Inf(1))) } } - return true + + return limitsPerQueuePerPc } diff --git a/internal/scheduler/scheduling/constraints/constraints_test.go b/internal/scheduler/scheduling/constraints/constraints_test.go index 42f33847bed..6d2cffcd18e 100644 --- a/internal/scheduler/scheduling/constraints/constraints_test.go +++ b/internal/scheduler/scheduling/constraints/constraints_test.go @@ -11,8 +11,9 @@ import ( "github.com/armadaproject/armada/internal/common/types" "github.com/armadaproject/armada/internal/scheduler/configuration" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/scheduling/context" + "github.com/armadaproject/armada/internal/scheduler/testfixtures" "github.com/armadaproject/armada/pkg/api" ) @@ -27,22 +28,32 @@ type constraintTest struct { } func TestConstraints(t *testing.T) { + rlFactory, err := internaltypes.NewResourceListFactory([]configuration.ResourceType{ + {Name: "cpu"}, + {Name: "memory"}, + {Name: "a"}, + {Name: "b"}, + {Name: "c"}, + {Name: "d"}, + }, nil) + assert.Nil(t, err) + tests := map[string]*constraintTest{ "no-constraints": makeConstraintsTest( - NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), + NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), makeSchedulingConfig(), - []*api.Queue{})), + []*api.Queue{{Name: "queue-1"}}, + ), rlFactory), "empty-queue-constraints": makeConstraintsTest( - NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), + NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), makeSchedulingConfig(), - []*api.Queue{{Name: "queue-1", Cordoned: false, ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{}}})), - "within-constraints": makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + []*api.Queue{{Name: "queue-1", Cordoned: false, ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{}}}), rlFactory), + "within-constraints": makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ MaximumResourceFractionToSchedule: map[string]float64{"cpu": 0.1, "memory": 0.1}, - MaxQueueLookback: 1000, PriorityClasses: map[string]types.PriorityClass{"priority-class-1": {MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{"pool-1": {"cpu": 0.9, "memory": 0.9}}}}, - }, []*api.Queue{{Name: "queue-1", Cordoned: false, ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{"priority-class-1": {MaximumResourceFraction: map[string]float64{"cpu": 0.9, "memory": 0.9}}}}})), + }, []*api.Queue{{Name: "queue-1", Cordoned: false, ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{"priority-class-1": {MaximumResourceFraction: map[string]float64{"cpu": 0.9, "memory": 0.9}}}}}), rlFactory), "exceeds-queue-priority-class-constraint": func() *constraintTest { - t := makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), makeSchedulingConfig(), []*api.Queue{ + t := makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), makeSchedulingConfig(), []*api.Queue{ { Name: "queue-1", Cordoned: false, @@ -52,12 +63,12 @@ func TestConstraints(t *testing.T) { }, }, }, - })) + }), rlFactory) t.expectedCheckConstraintsReason = "resource limit exceeded" return t }(), "exceeds-queue-priority-class-pool-constraint": func() *constraintTest { - t := makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), makeSchedulingConfig(), []*api.Queue{ + t := makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), makeSchedulingConfig(), []*api.Queue{ { Name: "queue-1", Cordoned: false, @@ -71,48 +82,51 @@ func TestConstraints(t *testing.T) { }, }, }, - })) + }), rlFactory) t.expectedCheckConstraintsReason = "resource limit exceeded" return t }(), "exceeds-priority-class-constraint": func() *constraintTest { - t := makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + t := makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ MaximumResourceFractionToSchedule: map[string]float64{"cpu": 0.1, "memory": 0.1}, - MaxQueueLookback: 1000, PriorityClasses: map[string]types.PriorityClass{"priority-class-1": {MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{"pool-1": {"cpu": 0.00000001, "memory": 0.9}}}}, - }, []*api.Queue{})) + }, []*api.Queue{{Name: "queue-1"}}), rlFactory) t.expectedCheckConstraintsReason = "resource limit exceeded" return t }(), - "priority-class-constraint-ignored-if-there-is-a-queue-constraint": makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + "priority-class-constraint-ignored-if-there-is-a-queue-constraint": makeConstraintsTest(NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ MaximumResourceFractionToSchedule: map[string]float64{"cpu": 0.1, "memory": 0.1}, - MaxQueueLookback: 1000, PriorityClasses: map[string]types.PriorityClass{"priority-class-1": {MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{"pool-1": {"cpu": 0.00000001, "memory": 0.9}}}}, - }, []*api.Queue{{Name: "queue-1", ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{"priority-class-1": {MaximumResourceFraction: map[string]float64{"cpu": 0.9, "memory": 0.9}}}}})), + }, []*api.Queue{{Name: "queue-1", ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{"priority-class-1": {MaximumResourceFraction: map[string]float64{"cpu": 0.9, "memory": 0.9}}}}}), rlFactory), "one-constraint-per-level-falls-back-as-expected--within-limits": makeMultiLevelConstraintsTest( map[string]resource.Quantity{"a": resource.MustParse("99"), "b": resource.MustParse("19"), "c": resource.MustParse("2.9"), "d": resource.MustParse("0.39")}, "", "", + rlFactory, ), "one-constraint-per-level-falls-back-as-expected--a-exceeds-limits": makeMultiLevelConstraintsTest( map[string]resource.Quantity{"a": resource.MustParse("101"), "b": resource.MustParse("19"), "c": resource.MustParse("2.9"), "d": resource.MustParse("0.39")}, UnschedulableReasonMaximumResourcesExceeded, "", + rlFactory, ), "one-constraint-per-level-falls-back-as-expected--b-exceeds-limits": makeMultiLevelConstraintsTest( map[string]resource.Quantity{"a": resource.MustParse("99"), "b": resource.MustParse("21"), "c": resource.MustParse("2.9"), "d": resource.MustParse("0.39")}, UnschedulableReasonMaximumResourcesExceeded, "", + rlFactory, ), "one-constraint-per-level-falls-back-as-expected--c-exceeds-limits": makeMultiLevelConstraintsTest( map[string]resource.Quantity{"a": resource.MustParse("99"), "b": resource.MustParse("19"), "c": resource.MustParse("3.1"), "d": resource.MustParse("0.39")}, UnschedulableReasonMaximumResourcesExceeded, "", + rlFactory, ), "one-constraint-per-level-falls-back-as-expected--d-exceeds-limits": makeMultiLevelConstraintsTest( map[string]resource.Quantity{"a": resource.MustParse("99"), "b": resource.MustParse("19"), "c": resource.MustParse("2.9"), "d": resource.MustParse("0.41")}, UnschedulableReasonMaximumResourcesExceeded, "", + rlFactory, ), } for name, tc := range tests { @@ -122,7 +136,7 @@ func TestConstraints(t *testing.T) { require.Equal(t, tc.expectedCheckRoundConstraintsReason == "", ok) require.Equal(t, tc.expectedCheckRoundConstraintsReason, unscheduledReason) - ok, unscheduledReason, err = tc.constraints.CheckConstraints(tc.sctx, tc.gctx) + ok, unscheduledReason, err = tc.constraints.CheckJobConstraints(tc.sctx, tc.gctx) require.NoError(t, err) require.Equal(t, tc.expectedCheckConstraintsReason == "", ok) require.Equal(t, tc.expectedCheckConstraintsReason, unscheduledReason) @@ -131,20 +145,21 @@ func TestConstraints(t *testing.T) { } func TestCapResources(t *testing.T) { + rlFactory := testfixtures.TestResourceListFactory tests := map[string]struct { constraints SchedulingConstraints queue string - resources schedulerobjects.QuantityByTAndResourceType[string] - expectedResources schedulerobjects.QuantityByTAndResourceType[string] + resources map[string]internaltypes.ResourceList + expectedResources map[string]internaltypes.ResourceList }{ "no contraints": { - constraints: NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), makeSchedulingConfig(), []*api.Queue{}), + constraints: NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), makeSchedulingConfig(), []*api.Queue{{Name: "queue-1"}}), queue: "queue-1", - resources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("1000", "1000Gi")}, - expectedResources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("1000", "1000Gi")}, + resources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "1000", "1000Gi")}, + expectedResources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "1000", "1000Gi")}, }, "unconstrained": { - constraints: NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + constraints: NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ PriorityClasses: map[string]types.PriorityClass{ "priority-class-1": { MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{ @@ -152,13 +167,13 @@ func TestCapResources(t *testing.T) { }, }, }, - }, []*api.Queue{}), + }, []*api.Queue{{Name: "queue-1"}}), queue: "queue-1", - resources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("1", "1Gi")}, - expectedResources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("1", "1Gi")}, + resources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "1", "1Gi")}, + expectedResources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "1", "1Gi")}, }, "per pool cap": { - constraints: NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + constraints: NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ PriorityClasses: map[string]types.PriorityClass{ "priority-class-1": { MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{ @@ -166,13 +181,13 @@ func TestCapResources(t *testing.T) { }, }, }, - }, []*api.Queue{}), + }, []*api.Queue{{Name: "queue-1"}}), queue: "queue-1", - resources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("1000", "1000Gi")}, - expectedResources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("100", "900Gi")}, + resources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "1000", "1000Gi")}, + expectedResources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "100", "900Gi")}, }, "per queue cap": { - constraints: NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + constraints: NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ PriorityClasses: map[string]types.PriorityClass{ "priority-class-1": { MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{ @@ -191,17 +206,18 @@ func TestCapResources(t *testing.T) { }, }), queue: "queue-1", - resources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("1000", "1000Gi")}, - expectedResources: map[string]schedulerobjects.ResourceList{"priority-class-1": makeResourceList("900", "900Gi")}, + resources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "1000", "1000Gi")}, + expectedResources: map[string]internaltypes.ResourceList{"priority-class-1": makeResourceList(rlFactory, "900", "900Gi")}, }, "per queue cap with multi pc": { - constraints: NewSchedulingConstraints("pool-1", makeResourceList("1000", "1000Gi"), configuration.SchedulingConfig{ + constraints: NewSchedulingConstraints("pool-1", makeResourceList(rlFactory, "1000", "1000Gi"), configuration.SchedulingConfig{ PriorityClasses: map[string]types.PriorityClass{ "priority-class-1": { MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{ "pool-1": {"cpu": 0.1, "memory": 0.9}, }, }, + "priority-class-2": {}, }, }, []*api.Queue{ { @@ -217,13 +233,13 @@ func TestCapResources(t *testing.T) { }, }), queue: "queue-1", - resources: map[string]schedulerobjects.ResourceList{ - "priority-class-1": makeResourceList("1000", "1000Gi"), - "priority-class-2": makeResourceList("2000", "2000Gi"), + resources: map[string]internaltypes.ResourceList{ + "priority-class-1": makeResourceList(rlFactory, "1000", "1000Gi"), + "priority-class-2": makeResourceList(rlFactory, "2000", "2000Gi"), }, - expectedResources: map[string]schedulerobjects.ResourceList{ - "priority-class-1": makeResourceList("100", "100Gi"), - "priority-class-2": makeResourceList("900", "900Gi"), + expectedResources: map[string]internaltypes.ResourceList{ + "priority-class-1": makeResourceList(rlFactory, "100", "100Gi"), + "priority-class-2": makeResourceList(rlFactory, "900", "900Gi"), }, }, } @@ -234,40 +250,33 @@ func TestCapResources(t *testing.T) { // Compare resources for equality. Note that we can't just do assert.Equal(tc.expectedResources, capped) // because the scale may have changed require.Equal(t, len(tc.expectedResources), len(capped), "number of priority classes differs") - for pc, rl := range tc.expectedResources { + for pc, expectedCappedRl := range tc.expectedResources { cappedRl, ok := capped[pc] require.True(t, ok, "no resource list found for priority class %s", pc) - require.Equal(t, len(rl.Resources), len(cappedRl.Resources), "number of resources differs for priority class %s", pc) - for res, qty := range rl.Resources { - cappedRes, ok := cappedRl.Resources[res] - require.True(t, ok, "resource %s doesn't exist at priority class %s", res, pc) - assert.Equal(t, 0, qty.Cmp(cappedRes), "resource %s differs at priority class %s", res, pc) - } + require.True(t, expectedCappedRl.Equal(cappedRl), "capped resources (%s) not as expected (%s)", cappedRl.String(), expectedCappedRl.String()) } }) } } -func makeMultiLevelConstraintsTest(requirements map[string]resource.Quantity, expectedCheckConstraintsReason string, expectedCheckRoundConstraintsReason string) *constraintTest { - zeroResources := schedulerobjects.ResourceList{ - Resources: map[string]resource.Quantity{"a": resource.MustParse("0"), "b": resource.MustParse("0"), "c": resource.MustParse("0"), "d": resource.MustParse("0")}, - } +func makeMultiLevelConstraintsTest(requirements map[string]resource.Quantity, expectedCheckConstraintsReason string, expectedCheckRoundConstraintsReason string, rlFactory *internaltypes.ResourceListFactory) *constraintTest { + rr := rlFactory.FromJobResourceListIgnoreUnknown(requirements) return &constraintTest{ - constraints: makeMultiLevelConstraints(), + constraints: makeMultiLevelConstraints(rlFactory), sctx: &context.SchedulingContext{ Pool: "pool-1", WeightSum: 100, - ScheduledResources: zeroResources.DeepCopy(), + ScheduledResources: internaltypes.ResourceList{}, Limiter: rate.NewLimiter(1e9, 1e6), QueueSchedulingContexts: map[string]*context.QueueSchedulingContext{ "queue-1": { Queue: "queue-1", Weight: 1, Limiter: rate.NewLimiter(1e9, 1e6), - Allocated: zeroResources.DeepCopy(), - AllocatedByPriorityClass: schedulerobjects.QuantityByTAndResourceType[string]{"priority-class-1": schedulerobjects.ResourceList{ - Resources: requirements, - }}, + Allocated: internaltypes.ResourceList{}, + AllocatedByPriorityClass: map[string]internaltypes.ResourceList{ + "priority-class-1": rr, + }, }, }, Started: time.Now(), @@ -277,7 +286,7 @@ func makeMultiLevelConstraintsTest(requirements map[string]resource.Quantity, ex PriorityClassName: "priority-class-1", }, Queue: "queue-1", - TotalResourceRequests: schedulerobjects.ResourceList{Resources: requirements}, + TotalResourceRequests: rr, JobSchedulingContexts: []*context.JobSchedulingContext{{}}, }, queue: "queue-1", @@ -287,89 +296,64 @@ func makeMultiLevelConstraintsTest(requirements map[string]resource.Quantity, ex } } -func makeMultiLevelConstraints() SchedulingConstraints { - return NewSchedulingConstraints("pool-1", schedulerobjects.ResourceList{ - Resources: map[string]resource.Quantity{ - "a": resource.MustParse("1000"), - "b": resource.MustParse("1000"), - "c": resource.MustParse("1000"), - "d": resource.MustParse("1000"), - }, - }, configuration.SchedulingConfig{ - MaxQueueLookback: 1000, - PriorityClasses: map[string]types.PriorityClass{ - "priority-class-1": { - MaximumResourceFractionPerQueue: map[string]float64{ - "a": 0.0001, "b": 0.0002, "c": 0.0003, "d": 0.0004, - }, - MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{ - "pool-1": { - "a": 0.001, "b": 0.002, "c": 0.003, - }, - }, +func makeMultiLevelConstraints(rlFactory *internaltypes.ResourceListFactory) SchedulingConstraints { + return NewSchedulingConstraints("pool-1", + rlFactory.FromNodeProto( + map[string]resource.Quantity{ + "a": resource.MustParse("1000"), + "b": resource.MustParse("1000"), + "c": resource.MustParse("1000"), + "d": resource.MustParse("1000"), }, - }, - }, []*api.Queue{ - { - Name: "queue-1", - ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{ + ), + configuration.SchedulingConfig{ + PriorityClasses: map[string]types.PriorityClass{ "priority-class-1": { - MaximumResourceFraction: map[string]float64{"a": 0.01, "b": 0.02}, - MaximumResourceFractionByPool: map[string]*api.PriorityClassPoolResourceLimits{ + MaximumResourceFractionPerQueue: map[string]float64{ + "a": 0.0001, "b": 0.0002, "c": 0.0003, "d": 0.0004, + }, + MaximumResourceFractionPerQueueByPool: map[string]map[string]float64{ "pool-1": { - MaximumResourceFraction: map[string]float64{"a": 0.1}, + "a": 0.001, "b": 0.002, "c": 0.003, + }, + }, + }, + }, + }, []*api.Queue{ + { + Name: "queue-1", + ResourceLimitsByPriorityClassName: map[string]*api.PriorityClassResourceLimits{ + "priority-class-1": { + MaximumResourceFraction: map[string]float64{"a": 0.01, "b": 0.02}, + MaximumResourceFractionByPool: map[string]*api.PriorityClassPoolResourceLimits{ + "pool-1": { + MaximumResourceFraction: map[string]float64{"a": 0.1}, + }, }, }, }, }, }, - }) -} - -func TestScaleQuantity(t *testing.T) { - tests := map[string]struct { - input resource.Quantity - f float64 - expected resource.Quantity - }{ - "one": { - input: resource.MustParse("1"), - f: 1, - expected: resource.MustParse("1"), - }, - "zero": { - input: resource.MustParse("1"), - f: 0, - expected: resource.MustParse("0"), - }, - "rounding": { - input: resource.MustParse("1"), - f: 0.3006, - expected: resource.MustParse("301m"), - }, - } - for name, tc := range tests { - t.Run(name, func(t *testing.T) { - assert.True(t, tc.expected.Equal(ScaleQuantity(tc.input, tc.f)), "expected %s, but got %s", tc.expected.String(), tc.input.String()) - }) - } + ) } -func makeConstraintsTest(constraints SchedulingConstraints) *constraintTest { +func makeConstraintsTest(constraints SchedulingConstraints, rlFactory *internaltypes.ResourceListFactory) *constraintTest { return &constraintTest{ constraints: constraints, sctx: &context.SchedulingContext{ Pool: "pool-1", WeightSum: 100, - ScheduledResources: makeResourceList("1", "1Gi"), + ScheduledResources: makeResourceList(rlFactory, "1", "1Gi"), Limiter: rate.NewLimiter(1e9, 1e6), QueueSchedulingContexts: map[string]*context.QueueSchedulingContext{ "queue-1": { - Queue: "queue-1", - Weight: 1, - Limiter: rate.NewLimiter(1e9, 1e6), - Allocated: makeResourceList("30", "1Gi"), - AllocatedByPriorityClass: schedulerobjects.QuantityByTAndResourceType[string]{"priority-class-1": makeResourceList("20", "1Gi")}, + Queue: "queue-1", + Weight: 1, + Limiter: rate.NewLimiter(1e9, 1e6), + Allocated: makeResourceList(rlFactory, "30", "1Gi"), + AllocatedByPriorityClass: map[string]internaltypes.ResourceList{ + "priority-class-1": makeResourceList(rlFactory, "20", "1Gi"), + }, }, }, Started: time.Now(), @@ -379,7 +363,7 @@ func makeConstraintsTest(constraints SchedulingConstraints) *constraintTest { PriorityClassName: "priority-class-1", }, Queue: "queue-1", - TotalResourceRequests: makeResourceList("1", "1Gi"), + TotalResourceRequests: makeResourceList(rlFactory, "1", "1Gi"), JobSchedulingContexts: []*context.JobSchedulingContext{{}}, }, queue: "queue-1", @@ -389,98 +373,18 @@ func makeConstraintsTest(constraints SchedulingConstraints) *constraintTest { } } -func TestIsStrictlyLessOrEqual(t *testing.T) { - tests := map[string]struct { - a map[string]resource.Quantity - b map[string]resource.Quantity - expected bool - }{ - "both empty": { - a: make(map[string]resource.Quantity), - b: make(map[string]resource.Quantity), - expected: true, - }, - "zero and missing is equal": { - a: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("0"), - }, - b: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - }, - expected: true, - }, - "simple equal": { - a: map[string]resource.Quantity{ - "cpu": resource.MustParse("1"), - "memory": resource.MustParse("2"), - "foo": resource.MustParse("3"), - }, - b: map[string]resource.Quantity{ - "cpu": resource.MustParse("1"), - "memory": resource.MustParse("2"), - "foo": resource.MustParse("3"), - }, - expected: true, - }, - "simple true": { - a: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("2"), - }, - b: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("3"), - }, - expected: true, - }, - "simple false": { - a: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("3"), - }, - b: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("2"), - }, - expected: false, - }, - "present in a missing in b true": { - a: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("2"), - }, - b: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - }, - expected: true, - }, - "missing in a present in b true": { - a: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - }, - b: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - "bar": resource.MustParse("2"), - }, - expected: true, - }, - } - for name, tc := range tests { - t.Run(name, func(t *testing.T) { - assert.Equal(t, tc.expected, isStrictlyLessOrEqual(tc.a, tc.b)) - }) - } -} - func makeSchedulingConfig() configuration.SchedulingConfig { return configuration.SchedulingConfig{ MaximumResourceFractionToSchedule: map[string]float64{"cpu": 0.1, "memory": 0.1}, - MaxQueueLookback: 1000, PriorityClasses: map[string]types.PriorityClass{"priority-class-1": {}}, } } -func makeResourceList(cpu string, memory string) schedulerobjects.ResourceList { - return schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"cpu": resource.MustParse(cpu), "memory": resource.MustParse(memory)}} +func makeResourceList(rlFactory *internaltypes.ResourceListFactory, cpu string, memory string) internaltypes.ResourceList { + return rlFactory.FromNodeProto( + map[string]resource.Quantity{ + "cpu": resource.MustParse(cpu), + "memory": resource.MustParse(memory), + }, + ) } diff --git a/internal/scheduler/scheduling/context/gang.go b/internal/scheduler/scheduling/context/gang.go index 0e6a58a9a6b..13cfddb4bba 100644 --- a/internal/scheduler/scheduling/context/gang.go +++ b/internal/scheduler/scheduling/context/gang.go @@ -3,7 +3,7 @@ package context import ( "time" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" ) type GangSchedulingContext struct { @@ -11,18 +11,18 @@ type GangSchedulingContext struct { Queue string GangInfo JobSchedulingContexts []*JobSchedulingContext - TotalResourceRequests schedulerobjects.ResourceList + TotalResourceRequests internaltypes.ResourceList AllJobsEvicted bool RequestsFloatingResources bool } func NewGangSchedulingContext(jctxs []*JobSchedulingContext) *GangSchedulingContext { allJobsEvicted := true - totalResourceRequests := schedulerobjects.NewResourceList(4) + totalResourceRequests := internaltypes.ResourceList{} requestsFloatingResources := false for _, jctx := range jctxs { allJobsEvicted = allJobsEvicted && jctx.IsEvicted - totalResourceRequests.AddV1ResourceList(jctx.PodRequirements.ResourceRequirements.Requests) + totalResourceRequests = totalResourceRequests.Add(jctx.Job.AllResourceRequirements()) if jctx.Job.RequestsFloatingResources() { requestsFloatingResources = true } diff --git a/internal/scheduler/scheduling/context/gang_test.go b/internal/scheduler/scheduling/context/gang_test.go index 88812e7beea..405826ca6ff 100644 --- a/internal/scheduler/scheduling/context/gang_test.go +++ b/internal/scheduler/scheduling/context/gang_test.go @@ -6,7 +6,6 @@ import ( "github.com/stretchr/testify/assert" "k8s.io/apimachinery/pkg/api/resource" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" "github.com/armadaproject/armada/internal/scheduler/testfixtures" ) @@ -18,12 +17,12 @@ func TestNewGangSchedulingContext(t *testing.T) { assert.Equal(t, testfixtures.TestDefaultPriorityClass, gctx.GangInfo.PriorityClassName) assert.True( t, - schedulerobjects.ResourceList{ - Resources: map[string]resource.Quantity{ + testfixtures.TestResourceListFactory.FromJobResourceListIgnoreUnknown( + map[string]resource.Quantity{ "cpu": resource.MustParse("2"), "memory": resource.MustParse("8Gi"), }, - }.Equal( + ).Equal( gctx.TotalResourceRequests, ), ) diff --git a/internal/scheduler/scheduling/context/queue.go b/internal/scheduler/scheduling/context/queue.go index cd09b23b128..b4707029f3b 100644 --- a/internal/scheduler/scheduling/context/queue.go +++ b/internal/scheduler/scheduling/context/queue.go @@ -12,6 +12,7 @@ import ( "golang.org/x/time/rate" armadaslices "github.com/armadaproject/armada/internal/common/slices" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" ) @@ -32,23 +33,20 @@ type QueueSchedulingContext struct { Limiter *rate.Limiter // Total resources assigned to the queue across all clusters by priority class priority. // Includes jobs scheduled during this invocation of the scheduler. - Allocated schedulerobjects.ResourceList + Allocated internaltypes.ResourceList // Total demand from this queue. This is essentially the cumulative resources of all non-terminal jobs at the // start of the scheduling cycle - Demand schedulerobjects.ResourceList + Demand internaltypes.ResourceList // Capped Demand for this queue. This differs from Demand in that it takes into account any limits that we have // placed on the queue - CappedDemand schedulerobjects.ResourceList + CappedDemand internaltypes.ResourceList // Fair share is the weight of this queue over the sum of the weights of all queues FairShare float64 // AdjustedFairShare modifies fair share such that queues that have a demand cost less than their fair share, have their fair share reallocated. AdjustedFairShare float64 // Total resources assigned to the queue across all clusters by priority class. // Includes jobs scheduled during this invocation of the scheduler. - AllocatedByPriorityClass schedulerobjects.QuantityByTAndResourceType[string] - // Total away resources assigned to the queue across all clusters by priority class. - // Includes away jobs scheduled during this invocation of the scheduler. - AwayAllocatedByPriorityClass schedulerobjects.QuantityByTAndResourceType[string] + AllocatedByPriorityClass map[string]internaltypes.ResourceList // Resources assigned to this queue during this scheduling cycle. ScheduledResourcesByPriorityClass schedulerobjects.QuantityByTAndResourceType[string] // Resources evicted from this queue during this scheduling cycle. @@ -66,7 +64,7 @@ func (qctx *QueueSchedulingContext) String() string { } // GetAllocation is necessary to implement the fairness.Queue interface. -func (qctx *QueueSchedulingContext) GetAllocation() schedulerobjects.ResourceList { +func (qctx *QueueSchedulingContext) GetAllocation() internaltypes.ResourceList { return qctx.Allocated } @@ -89,8 +87,10 @@ func (qctx *QueueSchedulingContext) ReportString(verbosity int32) string { fmt.Fprintf(w, "Preempted resources:\t%s\n", qctx.EvictedResourcesByPriorityClass.AggregateByResource().CompactString()) fmt.Fprintf(w, "Preempted resources (by priority):\t%s\n", qctx.EvictedResourcesByPriorityClass.String()) if verbosity >= 0 { - fmt.Fprintf(w, "Total allocated resources after scheduling:\t%s\n", qctx.Allocated.CompactString()) - fmt.Fprintf(w, "Total allocated resources after scheduling by priority class:\t%s\n", qctx.AllocatedByPriorityClass) + fmt.Fprintf(w, "Total allocated resources after scheduling:\t%s\n", qctx.Allocated.String()) + for pc, res := range qctx.AllocatedByPriorityClass { + fmt.Fprintf(w, "Total allocated resources after scheduling by for priority class %s:\t%s\n", pc, res.String()) + } fmt.Fprintf(w, "Number of jobs scheduled:\t%d\n", len(qctx.SuccessfulJobSchedulingContexts)) fmt.Fprintf(w, "Number of jobs preempted:\t%d\n", len(qctx.EvictedJobsById)) fmt.Fprintf(w, "Number of jobs that could not be scheduled:\t%d\n", len(qctx.UnsuccessfulJobSchedulingContexts)) @@ -170,8 +170,9 @@ func (qctx *QueueSchedulingContext) addJobSchedulingContext(jctx *JobSchedulingC // Always update ResourcesByPriority. // Since ResourcesByPriority is used to order queues by fraction of fair share. - qctx.Allocated.AddV1ResourceList(jctx.PodRequirements.ResourceRequirements.Requests) - qctx.AllocatedByPriorityClass.AddV1ResourceList(jctx.Job.PriorityClassName(), jctx.PodRequirements.ResourceRequirements.Requests) + pcName := jctx.Job.PriorityClassName() + qctx.AllocatedByPriorityClass[pcName] = qctx.AllocatedByPriorityClass[pcName].Add(jctx.Job.AllResourceRequirements()) + qctx.Allocated = qctx.Allocated.Add(jctx.Job.AllResourceRequirements()) // Only if the job is not evicted, update ScheduledResourcesByPriority. // Since ScheduledResourcesByPriority is used to control per-round scheduling constraints. @@ -205,8 +206,10 @@ func (qctx *QueueSchedulingContext) evictJob(job *jobdb.Job) (bool, error) { qctx.EvictedResourcesByPriorityClass.AddV1ResourceList(job.PriorityClassName(), rl) qctx.EvictedJobsById[jobId] = true } - qctx.Allocated.SubV1ResourceList(rl) - qctx.AllocatedByPriorityClass.SubV1ResourceList(job.PriorityClassName(), rl) + pcName := job.PriorityClassName() + qctx.AllocatedByPriorityClass[pcName] = qctx.AllocatedByPriorityClass[pcName].Subtract(job.AllResourceRequirements()) + qctx.Allocated = qctx.Allocated.Subtract(job.AllResourceRequirements()) + return scheduledInThisRound, nil } diff --git a/internal/scheduler/scheduling/context/scheduling.go b/internal/scheduler/scheduling/context/scheduling.go index e3bb7ae9d32..0feb303ba73 100644 --- a/internal/scheduler/scheduling/context/scheduling.go +++ b/internal/scheduler/scheduling/context/scheduling.go @@ -14,6 +14,7 @@ import ( "github.com/armadaproject/armada/internal/common/armadaerrors" armadamaps "github.com/armadaproject/armada/internal/common/maps" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" "github.com/armadaproject/armada/internal/scheduler/scheduling/fairness" @@ -37,11 +38,11 @@ type SchedulingContext struct { // Per-queue scheduling contexts. QueueSchedulingContexts map[string]*QueueSchedulingContext // Total resources across all clusters in this pool available at the start of the scheduling cycle. - TotalResources schedulerobjects.ResourceList + TotalResources internaltypes.ResourceList // Allocated resources across all clusters in this pool - Allocated schedulerobjects.ResourceList + Allocated internaltypes.ResourceList // Resources assigned across all queues during this scheduling cycle. - ScheduledResources schedulerobjects.ResourceList + ScheduledResources internaltypes.ResourceList ScheduledResourcesByPriorityClass schedulerobjects.QuantityByTAndResourceType[string] // Resources evicted across all queues during this scheduling cycle. EvictedResources schedulerobjects.ResourceList @@ -67,7 +68,7 @@ func NewSchedulingContext( pool string, fairnessCostProvider fairness.FairnessCostProvider, limiter *rate.Limiter, - totalResources schedulerobjects.ResourceList, + totalResources internaltypes.ResourceList, ) *SchedulingContext { return &SchedulingContext{ Started: time.Now(), @@ -75,8 +76,8 @@ func NewSchedulingContext( FairnessCostProvider: fairnessCostProvider, Limiter: limiter, QueueSchedulingContexts: make(map[string]*QueueSchedulingContext), - TotalResources: totalResources.DeepCopy(), - ScheduledResources: schedulerobjects.NewResourceListWithDefaultSize(), + TotalResources: totalResources, + ScheduledResources: internaltypes.ResourceList{}, ScheduledResourcesByPriorityClass: make(schedulerobjects.QuantityByTAndResourceType[string]), EvictedResourcesByPriorityClass: make(schedulerobjects.QuantityByTAndResourceType[string]), SchedulingKeyGenerator: schedulerobjects.NewSchedulingKeyGenerator(), @@ -90,9 +91,9 @@ func (sctx *SchedulingContext) ClearUnfeasibleSchedulingKeys() { func (sctx *SchedulingContext) AddQueueSchedulingContext( queue string, weight float64, - initialAllocatedByPriorityClass schedulerobjects.QuantityByTAndResourceType[string], - demand schedulerobjects.ResourceList, - cappedDemand schedulerobjects.ResourceList, + initialAllocatedByPriorityClass map[string]internaltypes.ResourceList, + demand internaltypes.ResourceList, + cappedDemand internaltypes.ResourceList, limiter *rate.Limiter, ) error { if _, ok := sctx.QueueSchedulingContexts[queue]; ok { @@ -103,16 +104,16 @@ func (sctx *SchedulingContext) AddQueueSchedulingContext( }) } if initialAllocatedByPriorityClass == nil { - initialAllocatedByPriorityClass = make(schedulerobjects.QuantityByTAndResourceType[string]) + initialAllocatedByPriorityClass = map[string]internaltypes.ResourceList{} } else { - initialAllocatedByPriorityClass = initialAllocatedByPriorityClass.DeepCopy() + initialAllocatedByPriorityClass = maps.Clone(initialAllocatedByPriorityClass) } - allocated := schedulerobjects.NewResourceListWithDefaultSize() + allocated := internaltypes.ResourceList{} for _, rl := range initialAllocatedByPriorityClass { - allocated.Add(rl) + allocated = allocated.Add(rl) } sctx.WeightSum += weight - sctx.Allocated.Add(allocated) + sctx.Allocated = sctx.Allocated.Add(allocated) qctx := &QueueSchedulingContext{ SchedulingContext: sctx, @@ -161,7 +162,7 @@ func (sctx *SchedulingContext) UpdateFairShares() { queueInfos := make([]*queueInfo, 0, len(sctx.QueueSchedulingContexts)) for queueName, qctx := range sctx.QueueSchedulingContexts { cappedShare := 1.0 - if !sctx.TotalResources.IsZero() { + if !sctx.TotalResources.AllZero() { cappedShare = sctx.FairnessCostProvider.UnweightedCostFromAllocation(qctx.CappedDemand) } queueInfos = append(queueInfos, &queueInfo{ @@ -218,8 +219,8 @@ func (sctx *SchedulingContext) ReportString(verbosity int32) string { fmt.Fprintf(w, "Finished:\t%s\n", sctx.Finished) fmt.Fprintf(w, "Duration:\t%s\n", sctx.Finished.Sub(sctx.Started)) fmt.Fprintf(w, "Termination reason:\t%s\n", sctx.TerminationReason) - fmt.Fprintf(w, "Total capacity:\t%s\n", sctx.TotalResources.CompactString()) - fmt.Fprintf(w, "Scheduled resources:\t%s\n", sctx.ScheduledResources.CompactString()) + fmt.Fprintf(w, "Total capacity:\t%s\n", sctx.TotalResources.String()) + fmt.Fprintf(w, "Scheduled resources:\t%s\n", sctx.ScheduledResources.String()) fmt.Fprintf(w, "Preempted resources:\t%s\n", sctx.EvictedResources.CompactString()) fmt.Fprintf(w, "Number of gangs scheduled:\t%d\n", sctx.NumScheduledGangs) fmt.Fprintf(w, "Number of jobs scheduled:\t%d\n", sctx.NumScheduledJobs) @@ -296,11 +297,11 @@ func (sctx *SchedulingContext) AddJobSchedulingContext(jctx *JobSchedulingContex sctx.EvictedResourcesByPriorityClass.SubV1ResourceList(jctx.Job.PriorityClassName(), jctx.PodRequirements.ResourceRequirements.Requests) sctx.NumEvictedJobs-- } else { - sctx.ScheduledResources.AddV1ResourceList(jctx.PodRequirements.ResourceRequirements.Requests) + sctx.ScheduledResources = sctx.ScheduledResources.Add(jctx.Job.AllResourceRequirements()) sctx.ScheduledResourcesByPriorityClass.AddV1ResourceList(jctx.Job.PriorityClassName(), jctx.PodRequirements.ResourceRequirements.Requests) sctx.NumScheduledJobs++ } - sctx.Allocated.AddV1ResourceList(jctx.PodRequirements.ResourceRequirements.Requests) + sctx.Allocated = sctx.Allocated.Add(jctx.Job.AllResourceRequirements()) } return evictedInThisRound, nil } @@ -341,7 +342,7 @@ func (sctx *SchedulingContext) EvictJob(jctx *JobSchedulingContext) (bool, error } rl := jctx.Job.ResourceRequirements().Requests if scheduledInThisRound { - sctx.ScheduledResources.SubV1ResourceList(rl) + sctx.ScheduledResources = sctx.ScheduledResources.Subtract(jctx.Job.AllResourceRequirements()) sctx.ScheduledResourcesByPriorityClass.SubV1ResourceList(jctx.Job.PriorityClassName(), rl) sctx.NumScheduledJobs-- } else { @@ -349,7 +350,7 @@ func (sctx *SchedulingContext) EvictJob(jctx *JobSchedulingContext) (bool, error sctx.EvictedResourcesByPriorityClass.AddV1ResourceList(jctx.Job.PriorityClassName(), rl) sctx.NumEvictedJobs++ } - sctx.Allocated.SubV1ResourceList(rl) + sctx.Allocated = sctx.Allocated.Subtract(jctx.Job.AllResourceRequirements()) return scheduledInThisRound, nil } @@ -371,14 +372,14 @@ func (sctx *SchedulingContext) SuccessfulJobSchedulingContexts() []*JobSchedulin } // AllocatedByQueueAndPriority returns map from queue name and priority to resources allocated. -func (sctx *SchedulingContext) AllocatedByQueueAndPriority() map[string]schedulerobjects.QuantityByTAndResourceType[string] { +func (sctx *SchedulingContext) AllocatedByQueueAndPriority() map[string]map[string]internaltypes.ResourceList { rv := make( - map[string]schedulerobjects.QuantityByTAndResourceType[string], + map[string]map[string]internaltypes.ResourceList, len(sctx.QueueSchedulingContexts), ) for queue, qctx := range sctx.QueueSchedulingContexts { - if !qctx.AllocatedByPriorityClass.IsZero() { - rv[queue] = qctx.AllocatedByPriorityClass.DeepCopy() + if !internaltypes.RlMapAllZero(qctx.AllocatedByPriorityClass) { + rv[queue] = maps.Clone(qctx.AllocatedByPriorityClass) } } return rv diff --git a/internal/scheduler/scheduling/context/scheduling_test.go b/internal/scheduler/scheduling/context/scheduling_test.go index 9c1334f8c6c..8031aa7540e 100644 --- a/internal/scheduler/scheduling/context/scheduling_test.go +++ b/internal/scheduler/scheduling/context/scheduling_test.go @@ -6,18 +6,18 @@ import ( "github.com/stretchr/testify/assert" "github.com/stretchr/testify/require" - "golang.org/x/exp/maps" "k8s.io/apimachinery/pkg/api/resource" - armadaslices "github.com/armadaproject/armada/internal/common/slices" "github.com/armadaproject/armada/internal/scheduler/configuration" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/scheduling/fairness" "github.com/armadaproject/armada/internal/scheduler/testfixtures" ) func TestSchedulingContextAccounting(t *testing.T) { - totalResources := schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"cpu": resource.MustParse("1")}} + totalResources := testfixtures.TestResourceListFactory.FromNodeProto( + map[string]resource.Quantity{"cpu": resource.MustParse("1")}, + ) fairnessCostProvider, err := fairness.NewDominantResourceFairness(totalResources, configuration.SchedulingConfig{DominantResourceFairnessResourcesToConsider: []string{"cpu"}}) require.NoError(t, err) sctx := NewSchedulingContext( @@ -27,13 +27,15 @@ func TestSchedulingContextAccounting(t *testing.T) { totalResources, ) priorityFactorByQueue := map[string]float64{"A": 1, "B": 1} - allocatedByQueueAndPriorityClass := map[string]schedulerobjects.QuantityByTAndResourceType[string]{ + allocatedByQueueAndPriorityClass := map[string]map[string]internaltypes.ResourceList{ "A": { - "foo": schedulerobjects.ResourceList{Resources: map[string]resource.Quantity{"cpu": resource.MustParse("1")}}, + testfixtures.TestDefaultPriorityClass: testfixtures.TestResourceListFactory.FromJobResourceListIgnoreUnknown( + map[string]resource.Quantity{"cpu": resource.MustParse("1")}, + ), }, } for _, queue := range []string{"A", "B"} { - err := sctx.AddQueueSchedulingContext(queue, priorityFactorByQueue[queue], allocatedByQueueAndPriorityClass[queue], schedulerobjects.ResourceList{}, schedulerobjects.ResourceList{}, nil) + err := sctx.AddQueueSchedulingContext(queue, priorityFactorByQueue[queue], allocatedByQueueAndPriorityClass[queue], internaltypes.ResourceList{}, internaltypes.ResourceList{}, nil) require.NoError(t, err) } @@ -42,18 +44,15 @@ func TestSchedulingContextAccounting(t *testing.T) { gctx := NewGangSchedulingContext(jctxs) _, err = sctx.AddGangSchedulingContext(gctx) require.NoError(t, err) + for _, jctx := range jctxs { _, err := sctx.EvictJob(jctx) require.NoError(t, err) } actual := sctx.AllocatedByQueueAndPriority() - queues := armadaslices.Unique( - armadaslices.Concatenate(maps.Keys(actual), maps.Keys(expected)), - ) - for _, queue := range queues { - assert.True(t, expected[queue].Equal(actual[queue])) - } + assert.Equal(t, expected, actual) + _, err = sctx.AddGangSchedulingContext(gctx) require.NoError(t, err) } @@ -65,7 +64,7 @@ func TestCalculateFairShares(t *testing.T) { oneHundredCpu := cpu(100) oneThousandCpu := cpu(1000) tests := map[string]struct { - availableResources schedulerobjects.ResourceList + availableResources internaltypes.ResourceList queueCtxs map[string]*QueueSchedulingContext expectedFairShares map[string]float64 expectedAdjustedFairShares map[string]float64 @@ -183,7 +182,7 @@ func TestCalculateFairShares(t *testing.T) { ) for qName, q := range tc.queueCtxs { err = sctx.AddQueueSchedulingContext( - qName, q.Weight, schedulerobjects.QuantityByTAndResourceType[string]{}, q.Demand, q.Demand, nil) + qName, q.Weight, map[string]internaltypes.ResourceList{}, q.Demand, q.Demand, nil) require.NoError(t, err) } sctx.UpdateFairShares() @@ -201,7 +200,7 @@ func TestCalculateFairShares(t *testing.T) { func TestCalculateFairnessError(t *testing.T) { tests := map[string]struct { - availableResources schedulerobjects.ResourceList + availableResources internaltypes.ResourceList queueCtxs map[string]*QueueSchedulingContext expected float64 }{ @@ -278,8 +277,8 @@ func testSmallCpuJobSchedulingContext(queue, priorityClassName string) *JobSched } } -func cpu(n int) schedulerobjects.ResourceList { - return schedulerobjects.ResourceList{ - Resources: map[string]resource.Quantity{"cpu": resource.MustParse(fmt.Sprintf("%d", n))}, - } +func cpu(n int) internaltypes.ResourceList { + return testfixtures.TestResourceListFactory.FromJobResourceListIgnoreUnknown( + map[string]resource.Quantity{"cpu": resource.MustParse(fmt.Sprintf("%d", n))}, + ) } diff --git a/internal/scheduler/scheduling/fairness/fairness.go b/internal/scheduler/scheduling/fairness/fairness.go index e71cdad2500..5072a3b4a94 100644 --- a/internal/scheduler/scheduling/fairness/fairness.go +++ b/internal/scheduler/scheduling/fairness/fairness.go @@ -4,11 +4,10 @@ import ( "fmt" "github.com/pkg/errors" - "k8s.io/apimachinery/pkg/api/resource" - "github.com/armadaproject/armada/internal/common/slices" + "github.com/armadaproject/armada/internal/common/maps" "github.com/armadaproject/armada/internal/scheduler/configuration" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" ) // QueueRepository is a minimal representation of a queue repository used for computing fairness. @@ -19,31 +18,32 @@ type QueueRepository interface { // Queue is a minimal representation of a queue used for computing fairness. type Queue interface { // GetAllocation returns the current allocation of the queue. - GetAllocation() schedulerobjects.ResourceList + GetAllocation() internaltypes.ResourceList GetWeight() float64 } // FairnessCostProvider captures algorithms to compute the cost of an allocation. type FairnessCostProvider interface { UnweightedCostFromQueue(queue Queue) float64 - UnweightedCostFromAllocation(allocation schedulerobjects.ResourceList) float64 + UnweightedCostFromAllocation(allocation internaltypes.ResourceList) float64 WeightedCostFromQueue(queue Queue) float64 - WeightedCostFromAllocation(allocation schedulerobjects.ResourceList, weight float64) float64 -} - -type resourceToConsider struct { - Name string - Multiplier float64 + WeightedCostFromAllocation(allocation internaltypes.ResourceList, weight float64) float64 } type DominantResourceFairness struct { // Total resources across all nodes. - totalResources schedulerobjects.ResourceList - // Resources considered when computing DominantResourceFairness. - resourcesToConsider []resourceToConsider + totalResources internaltypes.ResourceList + // Weight (defined in config) for each resource. + // Typically 1.0 (we care about that resource when assigning costs), + // or 0.0 (we don't care). However other values are possible. + multipliers internaltypes.ResourceFractionList } -func NewDominantResourceFairness(totalResources schedulerobjects.ResourceList, config configuration.SchedulingConfig) (*DominantResourceFairness, error) { +func NewDominantResourceFairness(totalResources internaltypes.ResourceList, config configuration.SchedulingConfig) (*DominantResourceFairness, error) { + if totalResources.IsEmpty() { + return &DominantResourceFairness{}, nil + } + if len(config.DominantResourceFairnessResourcesToConsider) != 0 && len(config.ExperimentalDominantResourceFairnessResourcesToConsider) != 0 { return nil, errors.New("config error - only one of DominantResourceFairnessResourcesToConsider and ExperimentalDominantResourceFairnessResourcesToConsider should be set") } @@ -53,22 +53,26 @@ func NewDominantResourceFairness(totalResources schedulerobjects.ResourceList, c } } - var resourcesToConsider []resourceToConsider + var multipliers map[string]float64 if len(config.DominantResourceFairnessResourcesToConsider) > 0 { - resourcesToConsider = slices.Map(config.DominantResourceFairnessResourcesToConsider, func(n string) resourceToConsider { - return resourceToConsider{Name: n, Multiplier: 1} + multipliers = maps.FromSlice(config.DominantResourceFairnessResourcesToConsider, func(n string) string { + return n + }, func(n string) float64 { + return 1.0 }) } else if len(config.ExperimentalDominantResourceFairnessResourcesToConsider) > 0 { - resourcesToConsider = slices.Map(config.ExperimentalDominantResourceFairnessResourcesToConsider, func(r configuration.DominantResourceFairnessResource) resourceToConsider { - return resourceToConsider{Name: r.Name, Multiplier: defaultMultiplier(r.Multiplier)} + multipliers = maps.FromSlice(config.ExperimentalDominantResourceFairnessResourcesToConsider, func(r configuration.DominantResourceFairnessResource) string { + return r.Name + }, func(r configuration.DominantResourceFairnessResource) float64 { + return defaultMultiplier(r.Multiplier) }) } else { return nil, errors.New("config error - DominantResourceFairnessResourcesToConsider and ExperimentalDominantResourceFairnessResourcesToConsider are both empty") } return &DominantResourceFairness{ - totalResources: totalResources, - resourcesToConsider: resourcesToConsider, + totalResources: totalResources, + multipliers: totalResources.Factory().MakeResourceFractionList(multipliers, 0.0), }, nil } @@ -87,23 +91,10 @@ func (f *DominantResourceFairness) UnweightedCostFromQueue(queue Queue) float64 return f.UnweightedCostFromAllocation(queue.GetAllocation()) } -func (f *DominantResourceFairness) WeightedCostFromAllocation(allocation schedulerobjects.ResourceList, weight float64) float64 { +func (f *DominantResourceFairness) WeightedCostFromAllocation(allocation internaltypes.ResourceList, weight float64) float64 { return f.UnweightedCostFromAllocation(allocation) / weight } -func (f *DominantResourceFairness) UnweightedCostFromAllocation(allocation schedulerobjects.ResourceList) float64 { - var cost float64 - for _, t := range f.resourcesToConsider { - capacity := f.totalResources.Get(t.Name) - if capacity.Equal(resource.Quantity{}) { - // Ignore any resources with zero capacity. - continue - } - q := allocation.Get(t.Name) - tcost := q.AsApproximateFloat64() / capacity.AsApproximateFloat64() * t.Multiplier - if tcost > cost { - cost = tcost - } - } - return cost +func (f *DominantResourceFairness) UnweightedCostFromAllocation(allocation internaltypes.ResourceList) float64 { + return max(0, allocation.DivideZeroOnError(f.totalResources).Multiply(f.multipliers).Max()) } diff --git a/internal/scheduler/scheduling/fairness/fairness_test.go b/internal/scheduler/scheduling/fairness/fairness_test.go index d5992092546..16cc4b4554e 100644 --- a/internal/scheduler/scheduling/fairness/fairness_test.go +++ b/internal/scheduler/scheduling/fairness/fairness_test.go @@ -8,15 +8,16 @@ import ( "k8s.io/apimachinery/pkg/api/resource" "github.com/armadaproject/armada/internal/scheduler/configuration" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" ) type MinimalQueue struct { - allocation schedulerobjects.ResourceList + allocation internaltypes.ResourceList weight float64 } -func (q MinimalQueue) GetAllocation() schedulerobjects.ResourceList { +func (q MinimalQueue) GetAllocation() internaltypes.ResourceList { return q.allocation } @@ -25,12 +26,12 @@ func (q MinimalQueue) GetWeight() float64 { } func TestNewDominantResourceFairness(t *testing.T) { + rlFactory := makeTestResourceListFactory() _, err := NewDominantResourceFairness( - schedulerobjects.ResourceList{ - Resources: map[string]resource.Quantity{ - "foo": resource.MustParse("1"), - }, + rlFactory.FromNodeProto(map[string]resource.Quantity{ + "foo": resource.MustParse("1"), }, + ), configuration.SchedulingConfig{DominantResourceFairnessResourcesToConsider: []string{}}, ) require.Error(t, err) @@ -204,20 +205,41 @@ func TestDominantResourceFairness(t *testing.T) { expectedCost: 2, }, } + + rlFactory := makeTestResourceListFactory() + for name, tc := range tests { t.Run(name, func(t *testing.T) { - f, err := NewDominantResourceFairness(tc.totalResources, tc.config) + totalResources := rlFactory.FromNodeProto(tc.totalResources.Resources) + allocation := rlFactory.FromJobResourceListIgnoreUnknown(tc.allocation.Resources) + f, err := NewDominantResourceFairness(totalResources, tc.config) require.NoError(t, err) assert.Equal( t, tc.expectedCost, - f.WeightedCostFromAllocation(tc.allocation, tc.weight), + f.WeightedCostFromAllocation(allocation, tc.weight), ) assert.Equal( t, - f.WeightedCostFromAllocation(tc.allocation, tc.weight), - f.WeightedCostFromQueue(MinimalQueue{allocation: tc.allocation, weight: tc.weight}), + f.WeightedCostFromAllocation(allocation, tc.weight), + f.WeightedCostFromQueue(MinimalQueue{allocation: allocation, weight: tc.weight}), ) }) } } + +func makeTestResourceListFactory() *internaltypes.ResourceListFactory { + rlFactory, err := internaltypes.NewResourceListFactory( + []configuration.ResourceType{ + {Name: "foo"}, + {Name: "bar"}, + }, + []configuration.FloatingResourceConfig{ + {Name: "baz"}, + }, + ) + if err != nil { + panic(err) + } + return rlFactory +} diff --git a/internal/scheduler/scheduling/gang_scheduler.go b/internal/scheduler/scheduling/gang_scheduler.go index 247c83e9b77..e8e0c898f45 100644 --- a/internal/scheduler/scheduling/gang_scheduler.go +++ b/internal/scheduler/scheduling/gang_scheduler.go @@ -136,7 +136,7 @@ func (sch *GangScheduler) Schedule(ctx *armadacontext.Context, gctx *context.Gan gangAddedToSchedulingContext = true if !gctx.AllJobsEvicted { // Only perform these checks for new jobs to avoid preempting jobs if, e.g., MinimumJobSize changes. - if ok, unschedulableReason, err = sch.constraints.CheckConstraints(sch.schedulingContext, gctx); err != nil || !ok { + if ok, unschedulableReason, err = sch.constraints.CheckJobConstraints(sch.schedulingContext, gctx); err != nil || !ok { return } } diff --git a/internal/scheduler/scheduling/gang_scheduler_test.go b/internal/scheduler/scheduling/gang_scheduler_test.go index e7f2dd17e77..d1626839282 100644 --- a/internal/scheduler/scheduling/gang_scheduler_test.go +++ b/internal/scheduler/scheduling/gang_scheduler_test.go @@ -18,6 +18,7 @@ import ( "github.com/armadaproject/armada/internal/common/util" "github.com/armadaproject/armada/internal/scheduler/configuration" "github.com/armadaproject/armada/internal/scheduler/floatingresources" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" "github.com/armadaproject/armada/internal/scheduler/nodedb" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" @@ -25,6 +26,7 @@ import ( "github.com/armadaproject/armada/internal/scheduler/scheduling/context" "github.com/armadaproject/armada/internal/scheduler/scheduling/fairness" "github.com/armadaproject/armada/internal/scheduler/testfixtures" + "github.com/armadaproject/armada/pkg/api" ) func TestGangScheduler(t *testing.T) { @@ -605,7 +607,9 @@ func TestGangScheduler(t *testing.T) { txn.Commit() if tc.TotalResources.Resources == nil { // Default to NodeDb total. - tc.TotalResources = nodeDb.TotalKubernetesResources() + tc.TotalResources = schedulerobjects.ResourceList{ + Resources: nodeDb.TotalKubernetesResources().ToMap(), + } } priorityFactorByQueue := make(map[string]float64) for _, jobs := range tc.Gangs { @@ -614,8 +618,10 @@ func TestGangScheduler(t *testing.T) { } } + totalResources := testfixtures.TestResourceListFactory.FromNodeProto(tc.TotalResources.Resources) + fairnessCostProvider, err := fairness.NewDominantResourceFairness( - tc.TotalResources, + totalResources, tc.SchedulingConfig, ) require.NoError(t, err) @@ -626,15 +632,15 @@ func TestGangScheduler(t *testing.T) { rate.Limit(tc.SchedulingConfig.MaximumSchedulingRate), tc.SchedulingConfig.MaximumSchedulingBurst, ), - tc.TotalResources, + totalResources, ) for queue, priorityFactor := range priorityFactorByQueue { err := sctx.AddQueueSchedulingContext( queue, priorityFactor, nil, - schedulerobjects.NewResourceList(0), - schedulerobjects.NewResourceList(0), + internaltypes.ResourceList{}, + internaltypes.ResourceList{}, rate.NewLimiter( rate.Limit(tc.SchedulingConfig.MaximumPerQueueSchedulingRate), tc.SchedulingConfig.MaximumPerQueueSchedulingBurst, @@ -642,8 +648,16 @@ func TestGangScheduler(t *testing.T) { ) require.NoError(t, err) } - constraints := schedulerconstraints.NewSchedulingConstraints("pool", tc.TotalResources, tc.SchedulingConfig, nil) - floatingResourceTypes, err := floatingresources.NewFloatingResourceTypes(tc.SchedulingConfig.ExperimentalFloatingResources) + constraints := schedulerconstraints.NewSchedulingConstraints( + "pool", + totalResources, + tc.SchedulingConfig, + armadaslices.Map( + maps.Keys(priorityFactorByQueue), + func(qn string) *api.Queue { return &api.Queue{Name: qn} }, + ), + ) + floatingResourceTypes, err := floatingresources.NewFloatingResourceTypes(tc.SchedulingConfig.ExperimentalFloatingResources, testfixtures.TestResourceListFactory) require.NoError(t, err) sch, err := NewGangScheduler(sctx, constraints, floatingResourceTypes, nodeDb, false) require.NoError(t, err) diff --git a/internal/scheduler/scheduling/preempting_queue_scheduler.go b/internal/scheduler/scheduling/preempting_queue_scheduler.go index 34d9d36ae9f..dec7c2422cd 100644 --- a/internal/scheduler/scheduling/preempting_queue_scheduler.go +++ b/internal/scheduler/scheduling/preempting_queue_scheduler.go @@ -14,9 +14,9 @@ import ( armadamaps "github.com/armadaproject/armada/internal/common/maps" armadaslices "github.com/armadaproject/armada/internal/common/slices" "github.com/armadaproject/armada/internal/scheduler/floatingresources" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" "github.com/armadaproject/armada/internal/scheduler/nodedb" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" schedulerconstraints "github.com/armadaproject/armada/internal/scheduler/scheduling/constraints" schedulercontext "github.com/armadaproject/armada/internal/scheduler/scheduling/context" "github.com/armadaproject/armada/internal/scheduler/scheduling/fairness" @@ -407,8 +407,11 @@ func (sch *PreemptingQueueScheduler) setEvictedGangCardinality(evictorResult *Ev func (sch *PreemptingQueueScheduler) evictionAssertions(evictorResult *EvictorResult) error { for _, qctx := range sch.schedulingContext.QueueSchedulingContexts { - if !qctx.AllocatedByPriorityClass.IsStrictlyNonNegative() { - return errors.Errorf("negative allocation for queue %s after eviction: %s", qctx.Queue, qctx.AllocatedByPriorityClass) + if internaltypes.RlMapHasNegativeValues(qctx.AllocatedByPriorityClass) { + return errors.Errorf("negative allocation for queue %s after eviction: %s", + qctx.Queue, + internaltypes.RlMapToString(qctx.AllocatedByPriorityClass), + ) } } evictedJobIdsByGangId := make(map[string]map[string]bool) @@ -454,17 +457,17 @@ func (qr *MinimalQueueRepository) GetQueue(name string) (fairness.Queue, bool) { func NewMinimalQueueRepositoryFromSchedulingContext(sctx *schedulercontext.SchedulingContext) *MinimalQueueRepository { queues := make(map[string]MinimalQueue, len(sctx.QueueSchedulingContexts)) for name, qctx := range sctx.QueueSchedulingContexts { - queues[name] = MinimalQueue{allocation: qctx.Allocated.DeepCopy(), weight: qctx.Weight} + queues[name] = MinimalQueue{allocation: qctx.Allocated, weight: qctx.Weight} } return &MinimalQueueRepository{queues: queues} } type MinimalQueue struct { - allocation schedulerobjects.ResourceList + allocation internaltypes.ResourceList weight float64 } -func (q MinimalQueue) GetAllocation() schedulerobjects.ResourceList { +func (q MinimalQueue) GetAllocation() internaltypes.ResourceList { return q.allocation } diff --git a/internal/scheduler/scheduling/preempting_queue_scheduler_test.go b/internal/scheduler/scheduling/preempting_queue_scheduler_test.go index 1d2da7c4dcf..eb0a3bb05ad 100644 --- a/internal/scheduler/scheduling/preempting_queue_scheduler_test.go +++ b/internal/scheduler/scheduling/preempting_queue_scheduler_test.go @@ -29,6 +29,7 @@ import ( "github.com/armadaproject/armada/internal/scheduler/scheduling/context" "github.com/armadaproject/armada/internal/scheduler/scheduling/fairness" "github.com/armadaproject/armada/internal/scheduler/testfixtures" + "github.com/armadaproject/armada/pkg/api" ) type testQueueContextChecker struct { @@ -2048,17 +2049,25 @@ func TestPreemptingQueueScheduler(t *testing.T) { for queue, priorityFactor := range tc.PriorityFactorByQueue { weight := 1 / priorityFactor + queueDemand := testfixtures.TestResourceListFactory.FromJobResourceListIgnoreUnknown(demandByQueue[queue].Resources) err := sctx.AddQueueSchedulingContext( queue, weight, - allocatedByQueueAndPriorityClass[queue], - demandByQueue[queue], - demandByQueue[queue], + internaltypes.RlMapFromJobSchedulerObjects(allocatedByQueueAndPriorityClass[queue], testfixtures.TestResourceListFactory), + queueDemand, + queueDemand, limiterByQueue[queue], ) require.NoError(t, err) } - constraints := schedulerconstraints.NewSchedulingConstraints("pool", totalResources, tc.SchedulingConfig, nil) + constraints := schedulerconstraints.NewSchedulingConstraints( + "pool", + totalResources, + tc.SchedulingConfig, + armadaslices.Map( + maps.Keys(tc.PriorityFactorByQueue), + func(qn string) *api.Queue { return &api.Queue{Name: qn} }, + )) sctx.UpdateFairShares() sch := NewPreemptingQueueScheduler( sctx, @@ -2104,7 +2113,8 @@ func TestPreemptingQueueScheduler(t *testing.T) { ) } for queue, qctx := range sctx.QueueSchedulingContexts { - assert.True(t, qctx.AllocatedByPriorityClass.Equal(allocatedByQueueAndPriorityClass[queue])) + m := internaltypes.RlMapFromJobSchedulerObjects(allocatedByQueueAndPriorityClass[queue], testfixtures.TestResourceListFactory) + assert.Equal(t, internaltypes.RlMapRemoveZeros(m), internaltypes.RlMapRemoveZeros(qctx.AllocatedByPriorityClass)) } // Test that jobs are mapped to nodes correctly. @@ -2404,11 +2414,19 @@ func BenchmarkPreemptingQueueScheduler(b *testing.B) { ) for queue, priorityFactor := range priorityFactorByQueue { weight := 1 / priorityFactor - err := sctx.AddQueueSchedulingContext(queue, weight, make(schedulerobjects.QuantityByTAndResourceType[string]), - schedulerobjects.NewResourceList(0), schedulerobjects.NewResourceList(0), limiterByQueue[queue]) + err := sctx.AddQueueSchedulingContext(queue, weight, make(map[string]internaltypes.ResourceList), + internaltypes.ResourceList{}, internaltypes.ResourceList{}, limiterByQueue[queue]) require.NoError(b, err) } - constraints := schedulerconstraints.NewSchedulingConstraints(testfixtures.TestPool, nodeDb.TotalKubernetesResources(), tc.SchedulingConfig, nil) + constraints := schedulerconstraints.NewSchedulingConstraints( + testfixtures.TestPool, + nodeDb.TotalKubernetesResources(), + tc.SchedulingConfig, + armadaslices.Map( + maps.Keys(priorityFactorByQueue), + func(qn string) *api.Queue { return &api.Queue{Name: qn} }, + ), + ) sch := NewPreemptingQueueScheduler( sctx, constraints, @@ -2468,7 +2486,7 @@ func BenchmarkPreemptingQueueScheduler(b *testing.B) { for queue, priorityFactor := range priorityFactorByQueue { weight := 1 / priorityFactor err := sctx.AddQueueSchedulingContext(queue, weight, allocatedByQueueAndPriorityClass[queue], - schedulerobjects.NewResourceList(0), schedulerobjects.NewResourceList(0), limiterByQueue[queue]) + internaltypes.ResourceList{}, internaltypes.ResourceList{}, limiterByQueue[queue]) require.NoError(b, err) } sch := NewPreemptingQueueScheduler( diff --git a/internal/scheduler/scheduling/queue_scheduler.go b/internal/scheduler/scheduling/queue_scheduler.go index 6938922051a..96969993ede 100644 --- a/internal/scheduler/scheduling/queue_scheduler.go +++ b/internal/scheduler/scheduling/queue_scheduler.go @@ -12,6 +12,7 @@ import ( "github.com/armadaproject/armada/internal/common/armadacontext" armadamaps "github.com/armadaproject/armada/internal/common/maps" "github.com/armadaproject/armada/internal/scheduler/floatingresources" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/nodedb" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" schedulerconstraints "github.com/armadaproject/armada/internal/scheduler/scheduling/constraints" @@ -142,11 +143,11 @@ func (sch *QueueScheduler) Schedule(ctx *armadacontext.Context) (*SchedulerResul stats.LastGangScheduledQueuePosition = loopNumber queue, queueOK := sch.candidateGangIterator.queueRepository.GetQueue(gctx.Queue) if queueOK { - stats.LastGangScheduledResources = gctx.TotalResourceRequests.DeepCopy() - stats.LastGangScheduledQueueResources = queue.GetAllocation().DeepCopy() + stats.LastGangScheduledResources = gctx.TotalResourceRequests + stats.LastGangScheduledQueueResources = queue.GetAllocation() } else { - stats.LastGangScheduledResources = schedulerobjects.NewResourceListWithDefaultSize() - stats.LastGangScheduledQueueResources = schedulerobjects.NewResourceListWithDefaultSize() + stats.LastGangScheduledResources = internaltypes.ResourceList{} + stats.LastGangScheduledQueueResources = internaltypes.ResourceList{} } } @@ -179,8 +180,8 @@ func (sch *QueueScheduler) Schedule(ctx *armadacontext.Context) (*SchedulerResul s.LastGangScheduledSampleJobId, s.LastGangScheduledQueuePosition, s.LastGangScheduledQueueCost, - s.LastGangScheduledResources.CompactString(), - s.LastGangScheduledQueueResources.CompactString(), + s.LastGangScheduledResources.String(), + s.LastGangScheduledQueueResources.String(), s.Time.Seconds()) })) @@ -331,8 +332,6 @@ type CandidateGangIterator struct { // If, e.g., onlyYieldEvictedByQueue["A"] is true, // this iterator only yields gangs where all jobs are evicted for queue A. onlyYieldEvictedByQueue map[string]bool - // Reusable buffer to avoid allocations. - buffer schedulerobjects.ResourceList // Priority queue containing per-queue iterators. // Determines the order in which queues are processed. pq QueueCandidateGangIteratorPQ @@ -350,7 +349,6 @@ func NewCandidateGangIterator( queueRepository: queueRepository, fairnessCostProvider: fairnessCostProvider, onlyYieldEvictedByQueue: make(map[string]bool), - buffer: schedulerobjects.NewResourceListWithDefaultSize(), pq: QueueCandidateGangIteratorPQ{ considerPriority: considerPriority, items: make([]*QueueCandidateGangIteratorItem, 0, len(iteratorsByQueue)), @@ -485,10 +483,8 @@ func (it *CandidateGangIterator) queueCostWithGctx(gctx *schedulercontext.GangSc if !ok { return 0, errors.Errorf("unknown queue %s", gangQueue) } - it.buffer.Zero() - it.buffer.Add(queue.GetAllocation()) - it.buffer.Add(gctx.TotalResourceRequests) - return it.fairnessCostProvider.WeightedCostFromAllocation(it.buffer, queue.GetWeight()), nil + + return it.fairnessCostProvider.WeightedCostFromAllocation(queue.GetAllocation().Add(gctx.TotalResourceRequests), queue.GetWeight()), nil } // QueueCandidateGangIteratorPQ is a priority queue used by CandidateGangIterator to determine from which queue to schedule the next job. diff --git a/internal/scheduler/scheduling/queue_scheduler_test.go b/internal/scheduler/scheduling/queue_scheduler_test.go index c8e002422f0..f1ef18c9faa 100644 --- a/internal/scheduler/scheduling/queue_scheduler_test.go +++ b/internal/scheduler/scheduling/queue_scheduler_test.go @@ -16,6 +16,7 @@ import ( armadaslices "github.com/armadaproject/armada/internal/common/slices" "github.com/armadaproject/armada/internal/common/stringinterner" "github.com/armadaproject/armada/internal/scheduler/configuration" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" "github.com/armadaproject/armada/internal/scheduler/nodedb" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" @@ -484,7 +485,9 @@ func TestQueueScheduler(t *testing.T) { txn.Commit() if tc.TotalResources.Resources == nil { // Default to NodeDb total. - tc.TotalResources = nodeDb.TotalKubernetesResources() + tc.TotalResources = schedulerobjects.ResourceList{ + Resources: nodeDb.TotalKubernetesResources().ToMap(), + } } queueNameToQueue := map[string]*api.Queue{} @@ -504,8 +507,9 @@ func TestQueueScheduler(t *testing.T) { context.JobSchedulingContextsFromJobs(tc.Jobs), ) + totalResources := testfixtures.TestResourceListFactory.FromJobResourceListIgnoreUnknown(tc.TotalResources.Resources) fairnessCostProvider, err := fairness.NewDominantResourceFairness( - tc.TotalResources, + totalResources, tc.SchedulingConfig, ) require.NoError(t, err) @@ -516,15 +520,18 @@ func TestQueueScheduler(t *testing.T) { rate.Limit(tc.SchedulingConfig.MaximumSchedulingRate), tc.SchedulingConfig.MaximumSchedulingBurst, ), - tc.TotalResources, + totalResources, ) for _, q := range tc.Queues { weight := 1.0 / float64(q.PriorityFactor) err := sctx.AddQueueSchedulingContext( q.Name, weight, - tc.InitialAllocatedByQueueAndPriorityClass[q.Name], - schedulerobjects.NewResourceList(0), - schedulerobjects.NewResourceList(0), + internaltypes.RlMapFromJobSchedulerObjects( + tc.InitialAllocatedByQueueAndPriorityClass[q.Name], + testfixtures.TestResourceListFactory, + ), + internaltypes.ResourceList{}, + internaltypes.ResourceList{}, rate.NewLimiter( rate.Limit(tc.SchedulingConfig.MaximumPerQueueSchedulingRate), tc.SchedulingConfig.MaximumPerQueueSchedulingBurst, @@ -532,7 +539,7 @@ func TestQueueScheduler(t *testing.T) { ) require.NoError(t, err) } - constraints := schedulerconstraints.NewSchedulingConstraints("pool", tc.TotalResources, tc.SchedulingConfig, tc.Queues) + constraints := schedulerconstraints.NewSchedulingConstraints("pool", totalResources, tc.SchedulingConfig, tc.Queues) jobIteratorByQueue := make(map[string]JobContextIterator) for _, q := range tc.Queues { it := jobRepo.GetJobIterator(q.Name) diff --git a/internal/scheduler/scheduling/result.go b/internal/scheduler/scheduling/result.go index a8669a6d6a2..5bf209880fa 100644 --- a/internal/scheduler/scheduling/result.go +++ b/internal/scheduler/scheduling/result.go @@ -3,8 +3,8 @@ package scheduling import ( "time" + "github.com/armadaproject/armada/internal/scheduler/internaltypes" "github.com/armadaproject/armada/internal/scheduler/jobdb" - "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" "github.com/armadaproject/armada/internal/scheduler/scheduling/context" ) @@ -18,8 +18,8 @@ type QueueStats struct { LastGangScheduledSampleJobId string LastGangScheduledQueuePosition int LastGangScheduledQueueCost float64 - LastGangScheduledResources schedulerobjects.ResourceList - LastGangScheduledQueueResources schedulerobjects.ResourceList + LastGangScheduledResources internaltypes.ResourceList + LastGangScheduledQueueResources internaltypes.ResourceList Time time.Duration } diff --git a/internal/scheduler/scheduling/scheduling_algo.go b/internal/scheduler/scheduling/scheduling_algo.go index f4b29b18724..01ad8e37773 100644 --- a/internal/scheduler/scheduling/scheduling_algo.go +++ b/internal/scheduler/scheduling/scheduling_algo.go @@ -128,8 +128,8 @@ func (l *FairSchedulingAlgo) Schedule( ctx.Infof("Scheduling on pool %s with capacity %s %s", pool, - fsctx.nodeDb.TotalKubernetesResources().CompactString(), - l.floatingResourceTypes.GetTotalAvailableForPool(pool.Name).CompactString(), + fsctx.nodeDb.TotalKubernetesResources().String(), + l.floatingResourceTypes.GetTotalAvailableForPoolInternalTypes(pool.Name).String(), ) start := time.Now() @@ -182,10 +182,6 @@ type FairSchedulingAlgoContext struct { Txn *jobdb.Txn } -func (l *FairSchedulingAlgo) NewFairSchedulingAlgoContext(ctx *armadacontext.Context, txn *jobdb.Txn, pool configuration.PoolConfig) (*FairSchedulingAlgoContext, error) { - return l.newFairSchedulingAlgoContext(ctx, txn, pool) -} - func (l *FairSchedulingAlgo) newFairSchedulingAlgoContext(ctx *armadacontext.Context, txn *jobdb.Txn, pool configuration.PoolConfig) (*FairSchedulingAlgoContext, error) { executors, err := l.executorRepository.GetExecutors(ctx) if err != nil { @@ -281,12 +277,12 @@ func (l *FairSchedulingAlgo) newFairSchedulingAlgoContext(ctx *armadacontext.Con } totalResources := nodeDb.TotalKubernetesResources() - totalResources = l.floatingResourceTypes.AddTotalAvailableForPool(pool.Name, totalResources) + totalResources = totalResources.Add(l.floatingResourceTypes.GetTotalAvailableForPoolInternalTypes(pool.Name)) schedulingContext, err := l.constructSchedulingContext( pool.Name, totalResources, - jobSchedulingInfo.demandByQueue, + jobSchedulingInfo.demandByQueueAndPriorityClass, jobSchedulingInfo.allocatedByQueueAndPriorityClass, jobSchedulingInfo.awayAllocatedByQueueAndPriorityClass, queueByName) @@ -312,9 +308,9 @@ type jobSchedulingInfo struct { nodeIdByJobId map[string]string jobIdsByGangId map[string]map[string]bool gangIdByJobId map[string]string - demandByQueue map[string]schedulerobjects.QuantityByTAndResourceType[string] - allocatedByQueueAndPriorityClass map[string]schedulerobjects.QuantityByTAndResourceType[string] - awayAllocatedByQueueAndPriorityClass map[string]schedulerobjects.QuantityByTAndResourceType[string] + demandByQueueAndPriorityClass map[string]map[string]internaltypes.ResourceList + allocatedByQueueAndPriorityClass map[string]map[string]internaltypes.ResourceList + awayAllocatedByQueueAndPriorityClass map[string]map[string]internaltypes.ResourceList } func calculateJobSchedulingInfo(ctx *armadacontext.Context, activeExecutorsSet map[string]bool, @@ -325,9 +321,9 @@ func calculateJobSchedulingInfo(ctx *armadacontext.Context, activeExecutorsSet m nodeIdByJobId := make(map[string]string) jobIdsByGangId := make(map[string]map[string]bool) gangIdByJobId := make(map[string]string) - demandByQueue := make(map[string]schedulerobjects.QuantityByTAndResourceType[string]) - allocatedByQueueAndPriorityClass := make(map[string]schedulerobjects.QuantityByTAndResourceType[string]) - awayAllocatedByQueueAndPriorityClass := make(map[string]schedulerobjects.QuantityByTAndResourceType[string]) + demandByQueueAndPriorityClass := make(map[string]map[string]internaltypes.ResourceList) + allocatedByQueueAndPriorityClass := make(map[string]map[string]internaltypes.ResourceList) + awayAllocatedByQueueAndPriorityClass := make(map[string]map[string]internaltypes.ResourceList) for _, job := range jobs { if job.InTerminalState() { @@ -361,19 +357,15 @@ func calculateJobSchedulingInfo(ctx *armadacontext.Context, activeExecutorsSet m } if slices.Contains(pools, currentPool) { - queueResources, ok := demandByQueue[job.Queue()] + queueResources, ok := demandByQueueAndPriorityClass[job.Queue()] if !ok { - queueResources = schedulerobjects.QuantityByTAndResourceType[string]{} - demandByQueue[job.Queue()] = queueResources + queueResources = map[string]internaltypes.ResourceList{} + demandByQueueAndPriorityClass[job.Queue()] = queueResources } // Queued jobs should not be considered for paused queues, so demand := running if !queue.Cordoned || !job.Queued() { - pcResources, ok := queueResources[job.PriorityClassName()] - if !ok { - pcResources = schedulerobjects.NewResourceList(len(job.PodRequirements().ResourceRequirements.Requests)) - queueResources[job.PriorityClassName()] = pcResources - } - pcResources.AddV1ResourceList(job.PodRequirements().ResourceRequirements.Requests) + pcName := job.PriorityClassName() + queueResources[pcName] = queueResources[pcName].Add(job.AllResourceRequirements()) } } @@ -396,17 +388,17 @@ func calculateJobSchedulingInfo(ctx *armadacontext.Context, activeExecutorsSet m if pool == currentPool { allocation := allocatedByQueueAndPriorityClass[queue.Name] if allocation == nil { - allocation = make(schedulerobjects.QuantityByTAndResourceType[string]) + allocation = make(map[string]internaltypes.ResourceList) allocatedByQueueAndPriorityClass[queue.Name] = allocation } - allocation.AddV1ResourceList(job.PriorityClassName(), job.ResourceRequirements().Requests) + allocation[job.PriorityClassName()] = allocation[job.PriorityClassName()].Add(job.AllResourceRequirements()) } else if slices.Contains(awayAllocationPools, pool) { awayAllocation := awayAllocatedByQueueAndPriorityClass[queue.Name] if awayAllocation == nil { - awayAllocation = make(schedulerobjects.QuantityByTAndResourceType[string]) + awayAllocation = make(map[string]internaltypes.ResourceList) awayAllocatedByQueueAndPriorityClass[queue.Name] = awayAllocation } - awayAllocation.AddV1ResourceList(job.PriorityClassName(), job.ResourceRequirements().Requests) + awayAllocation[job.PriorityClassName()] = awayAllocation[job.PriorityClassName()].Add(job.AllResourceRequirements()) } } if _, present := jobsByPool[pool]; !present { @@ -436,7 +428,7 @@ func calculateJobSchedulingInfo(ctx *armadacontext.Context, activeExecutorsSet m nodeIdByJobId: nodeIdByJobId, jobIdsByGangId: jobIdsByGangId, gangIdByJobId: gangIdByJobId, - demandByQueue: demandByQueue, + demandByQueueAndPriorityClass: demandByQueueAndPriorityClass, allocatedByQueueAndPriorityClass: allocatedByQueueAndPriorityClass, awayAllocatedByQueueAndPriorityClass: awayAllocatedByQueueAndPriorityClass, }, nil @@ -463,10 +455,10 @@ func (l *FairSchedulingAlgo) constructNodeDb(homeJobs []*jobdb.Job, awayJobs []* func (l *FairSchedulingAlgo) constructSchedulingContext( pool string, - totalCapacity schedulerobjects.ResourceList, - demandByQueue map[string]schedulerobjects.QuantityByTAndResourceType[string], - allocationByQueueAndPriorityClass map[string]schedulerobjects.QuantityByTAndResourceType[string], - awayAllocationByQueueAndPriorityClass map[string]schedulerobjects.QuantityByTAndResourceType[string], + totalCapacity internaltypes.ResourceList, + demandByQueueAndPriorityClass map[string]map[string]internaltypes.ResourceList, + allocationByQueueAndPriorityClass map[string]map[string]internaltypes.ResourceList, + awayAllocationByQueueAndPriorityClass map[string]map[string]internaltypes.ResourceList, queues map[string]*api.Queue, ) (*schedulercontext.SchedulingContext, error) { fairnessCostProvider, err := fairness.NewDominantResourceFairness(totalCapacity, l.schedulingConfig) @@ -477,16 +469,16 @@ func (l *FairSchedulingAlgo) constructSchedulingContext( constraints := schedulerconstraints.NewSchedulingConstraints(pool, totalCapacity, l.schedulingConfig, maps.Values(queues)) for _, queue := range queues { - demand, hasDemand := demandByQueue[queue.Name] + demand, hasDemand := demandByQueueAndPriorityClass[queue.Name] if !hasDemand { // To ensure fair share is computed only from active queues, i.e., queues with jobs queued or running. continue } cappedDemand := constraints.CapResources(queue.Name, demand) - var allocatedByPriorityClass schedulerobjects.QuantityByTAndResourceType[string] - if allocatedByQueueAndPriorityClass := allocationByQueueAndPriorityClass; allocatedByQueueAndPriorityClass != nil { - allocatedByPriorityClass = allocatedByQueueAndPriorityClass[queue.Name] + var allocatedByPriorityClass map[string]internaltypes.ResourceList + if allocationByQueueAndPriorityClass != nil { + allocatedByPriorityClass = allocationByQueueAndPriorityClass[queue.Name] } var weight float64 = 1 if queue.PriorityFactor > 0 { @@ -502,7 +494,7 @@ func (l *FairSchedulingAlgo) constructSchedulingContext( l.limiterByQueue[queue.Name] = queueLimiter } - if err := sctx.AddQueueSchedulingContext(queue.Name, weight, allocatedByPriorityClass, demand.AggregateByResource(), cappedDemand.AggregateByResource(), queueLimiter); err != nil { + if err := sctx.AddQueueSchedulingContext(queue.Name, weight, allocatedByPriorityClass, internaltypes.RlMapSumValues(demand), internaltypes.RlMapSumValues(cappedDemand), queueLimiter); err != nil { return nil, err } } @@ -519,7 +511,7 @@ func (l *FairSchedulingAlgo) constructSchedulingContext( weight = 1 / queue.PriorityFactor } - if err := sctx.AddQueueSchedulingContext(schedulercontext.CalculateAwayQueueName(queue.Name), weight, allocation, schedulerobjects.NewResourceList(0), schedulerobjects.NewResourceList(0), nil); err != nil { + if err := sctx.AddQueueSchedulingContext(schedulercontext.CalculateAwayQueueName(queue.Name), weight, allocation, internaltypes.ResourceList{}, internaltypes.ResourceList{}, nil); err != nil { return nil, err } } @@ -536,7 +528,8 @@ func (l *FairSchedulingAlgo) SchedulePool( pool string, ) (*SchedulerResult, *schedulercontext.SchedulingContext, error) { totalResources := fsctx.nodeDb.TotalKubernetesResources() - totalResources = l.floatingResourceTypes.AddTotalAvailableForPool(pool, totalResources) + totalResources = totalResources.Add(l.floatingResourceTypes.GetTotalAvailableForPoolInternalTypes(pool)) + constraints := schedulerconstraints.NewSchedulingConstraints(pool, totalResources, l.schedulingConfig, maps.Values(fsctx.queues)) scheduler := NewPreemptingQueueScheduler( diff --git a/internal/scheduler/simulator/simulator.go b/internal/scheduler/simulator/simulator.go index aec3a3d55c2..dec3c5d406d 100644 --- a/internal/scheduler/simulator/simulator.go +++ b/internal/scheduler/simulator/simulator.go @@ -53,11 +53,11 @@ type accounting struct { nodeDbByPool map[string]*nodedb.NodeDb // Allocation by pool for each queue and priority class. // Stored across invocations of the scheduler. - allocationByPoolAndQueueAndPriorityClass map[string]map[string]schedulerobjects.QuantityByTAndResourceType[string] + allocationByPoolAndQueueAndPriorityClass map[string]map[string]map[string]internaltypes.ResourceList // Demand for each queue - demandByQueue map[string]schedulerobjects.ResourceList + demandByQueue map[string]internaltypes.ResourceList // Total resources across all executorGroups for each pool. - totalResourcesByPool map[string]schedulerobjects.ResourceList + totalResourcesByPool map[string]internaltypes.ResourceList // Mapping of job Id -> nodeId. Needed by preemptingqueuescheduler for gang preemption. nodeIdByJobId map[string]string // Mapping of gangId -> jobsINGang. Needed by preemptingqueuescheduler for gang preemption. @@ -132,7 +132,7 @@ func NewSimulator( return nil, errors.WithMessage(err, "Error with the .scheduling.supportedResourceTypes field in config") } - floatingResourceTypes, err := floatingresources.NewFloatingResourceTypes(schedulingConfig.ExperimentalFloatingResources) + floatingResourceTypes, err := floatingresources.NewFloatingResourceTypes(schedulingConfig.ExperimentalFloatingResources, resourceListFactory) if err != nil { return nil, err } @@ -176,9 +176,9 @@ func NewSimulator( accounting: accounting{ nodeDbByPool: make(map[string]*nodedb.NodeDb), poolByNodeId: make(map[string]string), - allocationByPoolAndQueueAndPriorityClass: make(map[string]map[string]schedulerobjects.QuantityByTAndResourceType[string]), - demandByQueue: make(map[string]schedulerobjects.ResourceList), - totalResourcesByPool: make(map[string]schedulerobjects.ResourceList), + allocationByPoolAndQueueAndPriorityClass: make(map[string]map[string]map[string]internaltypes.ResourceList), + demandByQueue: make(map[string]internaltypes.ResourceList), + totalResourcesByPool: make(map[string]internaltypes.ResourceList), nodeIdByJobId: make(map[string]string), jobIdsByGangId: make(map[string]map[string]bool), gangIdByJobId: make(map[string]string), @@ -338,11 +338,6 @@ func (s *Simulator) setupClusters() error { s.accounting.nodeDbByPool[cluster.Pool] = nodeDb } - totalResourcesForPool, ok := s.accounting.totalResourcesByPool[cluster.Pool] - if !ok { - totalResourcesForPool = schedulerobjects.ResourceList{} - } - for nodeTemplateIndex, nodeTemplate := range cluster.NodeTemplates { labels := map[string]string{} if nodeTemplate.Labels != nil { @@ -378,9 +373,13 @@ func (s *Simulator) setupClusters() error { s.accounting.poolByNodeId[nodeId] = cluster.Pool } } - totalResourcesForPool.Add(nodeDb.TotalKubernetesResources()) - s.accounting.totalResourcesByPool[cluster.Pool] = totalResourcesForPool + } + + for pool, nodeDb := range s.accounting.nodeDbByPool { + s.accounting.totalResourcesByPool[pool] = nodeDb.TotalKubernetesResources() + } + return nil } @@ -898,10 +897,8 @@ func (s *Simulator) handleJobSucceeded(txn *jobdb.Txn, e *armadaevents.JobSuccee // Subtract the allocation of this job from the queue allocation. run := job.LatestRun() pool := s.accounting.poolByNodeId[run.NodeId()] - s.accounting.allocationByPoolAndQueueAndPriorityClass[pool][job.Queue()].SubV1ResourceList( - job.PriorityClassName(), - job.ResourceRequirements().Requests, - ) + allocByPc := s.accounting.allocationByPoolAndQueueAndPriorityClass[pool][job.Queue()] + allocByPc[job.PriorityClassName()] = allocByPc[job.PriorityClassName()].Subtract(job.AllResourceRequirements()) s.removeJobFromDemand(job) // Unbind the job from the node on which it was scheduled. @@ -1036,19 +1033,11 @@ func maxTime(a, b time.Time) time.Time { } func (s *Simulator) addJobToDemand(job *jobdb.Job) { - r, ok := s.accounting.demandByQueue[job.Queue()] - if !ok { - r = schedulerobjects.NewResourceList(len(job.PodRequirements().ResourceRequirements.Requests)) - s.accounting.demandByQueue[job.Queue()] = r - } - r.AddV1ResourceList(job.PodRequirements().ResourceRequirements.Requests) + s.accounting.demandByQueue[job.Queue()] = s.accounting.demandByQueue[job.Queue()].Add(job.AllResourceRequirements()) } func (s *Simulator) removeJobFromDemand(job *jobdb.Job) { - r, ok := s.accounting.demandByQueue[job.Queue()] - if ok { - r.SubV1ResourceList(job.PodRequirements().ResourceRequirements.Requests) - } + s.accounting.demandByQueue[job.Queue()] = s.accounting.demandByQueue[job.Queue()].Subtract(job.AllResourceRequirements()) } func expandRepeatingTemplates(w *WorkloadSpec) *WorkloadSpec { diff --git a/internal/scheduler/simulator/sink/queue_stats_writer.go b/internal/scheduler/simulator/sink/queue_stats_writer.go index 07b575e6509..495d7d016bc 100644 --- a/internal/scheduler/simulator/sink/queue_stats_writer.go +++ b/internal/scheduler/simulator/sink/queue_stats_writer.go @@ -90,12 +90,12 @@ func (j *QueueStatsWriter) Close(ctx *armadacontext.Context) { } func calculateResourceShare(sctx *context.SchedulingContext, qctx *context.QueueSchedulingContext, resource string) float64 { - total := sctx.Allocated.Resources[resource] - allocated := qctx.Allocated.Resources[resource] + total := sctx.Allocated.GetResourceByNameZeroIfMissing(resource) + allocated := qctx.Allocated.GetResourceByNameZeroIfMissing(resource) return allocated.AsApproximateFloat64() / total.AsApproximateFloat64() } func allocatedResources(qctx *context.QueueSchedulingContext, resource string) int { - allocated := qctx.Allocated.Resources[resource] + allocated := qctx.Allocated.GetResourceByNameZeroIfMissing(resource) return int(allocated.AsApproximateFloat64()) } diff --git a/internal/scheduler/simulator/test_utils.go b/internal/scheduler/simulator/test_utils.go index 38e7a3d976f..d55b6437103 100644 --- a/internal/scheduler/simulator/test_utils.go +++ b/internal/scheduler/simulator/test_utils.go @@ -13,7 +13,6 @@ import ( "github.com/armadaproject/armada/internal/common/types" "github.com/armadaproject/armada/internal/scheduler/configuration" "github.com/armadaproject/armada/internal/scheduler/schedulerobjects" - "github.com/armadaproject/armada/internal/scheduler/scheduling/constraints" "github.com/armadaproject/armada/pkg/armadaevents" ) @@ -88,19 +87,6 @@ func GetBasicSchedulingConfig() configuration.SchedulingConfig { } } -// TotalResources returns the total resources available across all nodes in the ClusterSpec. -func (cs *ClusterSpec) TotalResources() schedulerobjects.ResourceList { - total := schedulerobjects.NewResourceListWithDefaultSize() - for _, cluster := range cs.Clusters { - for _, nt := range cluster.NodeTemplates { - for t, q := range nt.TotalResources.Resources { - total.AddQuantity(t, constraints.ScaleQuantity(q, float64(nt.Number))) - } - } - } - return total -} - func NodeTemplate32Cpu(n int64) *NodeTemplate { return &NodeTemplate{ Number: n, diff --git a/internal/scheduler/testfixtures/testfixtures.go b/internal/scheduler/testfixtures/testfixtures.go index 3b869f2ef49..e18c5c35073 100644 --- a/internal/scheduler/testfixtures/testfixtures.go +++ b/internal/scheduler/testfixtures/testfixtures.go @@ -947,7 +947,7 @@ func MakeTestResourceListFactory() *internaltypes.ResourceListFactory { } func MakeTestFloatingResourceTypes(config []schedulerconfiguration.FloatingResourceConfig) *floatingresources.FloatingResourceTypes { - result, _ := floatingresources.NewFloatingResourceTypes(config) + result, _ := floatingresources.NewFloatingResourceTypes(config, TestResourceListFactory) return result }