Documentation ¶
Index ¶
- func SchedulingKeyFromJob(skg *schedulerobjects.SchedulingKeyGenerator, job *Job) schedulerobjects.SchedulingKey
- func SchedulingOrderCompare(job, other *Job) int
- type Job
- func (job *Job) ActiveRunTimestamp() int64
- func (job *Job) Affinity() *v1.Affinity
- func (job *Job) AllRuns() []*JobRun
- func (job *Job) Annotations() map[string]string
- func (job *Job) Assert() error
- func (job *Job) CancelByJobsetRequested() bool
- func (job *Job) CancelRequested() bool
- func (job *Job) Cancelled() bool
- func (job *Job) Created() int64
- func (job *Job) DeepCopy() *Job
- func (job *Job) Equal(other *Job) bool
- func (job *Job) Failed() bool
- func (job *Job) HasQueueTtlExpired() bool
- func (job *Job) HasQueueTtlSet() bool
- func (job *Job) HasRuns() bool
- func (job *Job) Id() string
- func (job *Job) InTerminalState() bool
- func (job *Job) JobSchedulingInfo() *schedulerobjects.JobSchedulingInfo
- func (job *Job) Jobset() string
- func (job *Job) LatestRun() *JobRun
- func (job *Job) NodeSelector() map[string]string
- func (job *Job) NumAttempts() uint
- func (job *Job) NumReturned() uint
- func (job *Job) PodRequirements() *schedulerobjects.PodRequirements
- func (job *Job) Priority() uint32
- func (job *Job) PriorityClass() types.PriorityClass
- func (job *Job) PriorityClassName() string
- func (job *Job) Queue() string
- func (job *Job) QueueTtlSeconds() int64
- func (job *Job) Queued() bool
- func (job *Job) QueuedVersion() int32
- func (job *Job) RequestedPriority() uint32
- func (job *Job) ResourceRequirements() v1.ResourceRequirements
- func (job *Job) RunById(id uuid.UUID) *JobRun
- func (job *Job) ScheduledAtPriority() (int32, bool)
- func (job *Job) SchedulingKey() schedulerobjects.SchedulingKey
- func (job *Job) SchedulingOrderCompare(other *Job) int
- func (job *Job) String() string
- func (job *Job) SubmitTime() time.Time
- func (job *Job) Succeeded() bool
- func (job *Job) Tolerations() []v1.Toleration
- func (job *Job) Validated() bool
- func (job *Job) WithCancelByJobsetRequested(cancelByJobsetRequested bool) *Job
- func (job *Job) WithCancelRequested(cancelRequested bool) *Job
- func (job *Job) WithCancelled(cancelled bool) *Job
- func (job *Job) WithCreated(created int64) *Job
- func (job *Job) WithFailed(failed bool) *Job
- func (job *Job) WithJobSchedulingInfo(jobSchedulingInfo *schedulerobjects.JobSchedulingInfo) *Job
- func (job *Job) WithJobset(jobset string) *Job
- func (job *Job) WithNewRun(executor string, nodeId, nodeName string, scheduledAtPriority int32) *Job
- func (job *Job) WithPriority(priority uint32) *Job
- func (job *Job) WithPriorityClass(priorityClass types.PriorityClass) *Job
- func (job *Job) WithQueue(queue string) *Job
- func (job *Job) WithQueued(queued bool) *Job
- func (job *Job) WithQueuedVersion(version int32) *Job
- func (job *Job) WithRequestedPriority(priority uint32) *Job
- func (job *Job) WithSubmittedTime(submittedTime int64) *Job
- func (job *Job) WithSucceeded(succeeded bool) *Job
- func (job *Job) WithUpdatedRun(run *JobRun) *Job
- func (job *Job) WithValidated(validated bool) *Job
- func (job *Job) WithoutTerminal() *Job
- type JobDb
- func (jobDb *JobDb) Clone() *JobDb
- func (jobDb *JobDb) CreateRun(id uuid.UUID, jobId string, creationTime int64, executor string, nodeId string, ...) *JobRun
- func (jobDb *JobDb) NewJob(jobId string, jobSet string, queue string, priority uint32, ...) *Job
- func (jobDb *JobDb) ReadTxn() *Txn
- func (jobDb *JobDb) ReconcileDifferences(txn *Txn, jobRepoJobs []database.Job, jobRepoRuns []database.Run) ([]JobStateTransitions, error)
- func (jobDb *JobDb) SetClock(clock clock.PassiveClock)
- func (jobDb *JobDb) SetUUIDProvider(uuidProvider UUIDProvider)
- func (jobDb *JobDb) WriteTxn() *Txn
- type JobIdHasher
- type JobPriorityComparer
- type JobQueueTtlComparer
- type JobRun
- func (run *JobRun) Assert() error
- func (run *JobRun) Cancelled() bool
- func (run *JobRun) Created() int64
- func (run *JobRun) DeepCopy() *JobRun
- func (run *JobRun) Equal(other *JobRun) bool
- func (run *JobRun) Executor() string
- func (run *JobRun) Failed() bool
- func (run *JobRun) Id() uuid.UUID
- func (run *JobRun) InTerminalState() bool
- func (run *JobRun) JobId() string
- func (run *JobRun) LeaseTime() *time.Time
- func (run *JobRun) Leased() bool
- func (run *JobRun) NodeId() string
- func (run *JobRun) NodeName() string
- func (run *JobRun) Pending() bool
- func (run *JobRun) PendingTime() *time.Time
- func (run *JobRun) PreemptRequested() bool
- func (run *JobRun) Preempted() bool
- func (run *JobRun) PreemptedTime() *time.Time
- func (run *JobRun) Returned() bool
- func (run *JobRun) RunAttempted() bool
- func (run *JobRun) Running() bool
- func (run *JobRun) RunningTime() *time.Time
- func (run *JobRun) ScheduledAtPriority() *int32
- func (run *JobRun) String() string
- func (run *JobRun) Succeeded() bool
- func (run *JobRun) TerminatedTime() *time.Time
- func (run *JobRun) WithAttempted(attempted bool) *JobRun
- func (run *JobRun) WithCancelled(cancelled bool) *JobRun
- func (run *JobRun) WithFailed(failed bool) *JobRun
- func (run *JobRun) WithLeased(leased bool) *JobRun
- func (run *JobRun) WithLeasedTime(leaseTime *time.Time) *JobRun
- func (run *JobRun) WithPending(pending bool) *JobRun
- func (run *JobRun) WithPendingTime(pendingTime *time.Time) *JobRun
- func (run *JobRun) WithPreemptRequested(preemptRequested bool) *JobRun
- func (run *JobRun) WithPreempted(preempted bool) *JobRun
- func (run *JobRun) WithPreemptedTime(preemptedTime *time.Time) *JobRun
- func (run *JobRun) WithReturned(returned bool) *JobRun
- func (run *JobRun) WithRunning(running bool) *JobRun
- func (run *JobRun) WithRunningTime(runningTime *time.Time) *JobRun
- func (run *JobRun) WithSucceeded(succeeded bool) *JobRun
- func (run *JobRun) WithTerminatedTime(terminatedTime *time.Time) *JobRun
- func (run *JobRun) WithoutTerminal() *JobRun
- type JobStateTransitions
- type RealUUIDProvider
- type RunStateTransitions
- type Txn
- func (txn *Txn) Abort()
- func (txn *Txn) Assert(assertOnlyActiveJobs bool) error
- func (txn *Txn) AssertEqual(otherTxn *Txn) error
- func (txn *Txn) BatchDelete(jobIds []string) error
- func (txn *Txn) Commit()
- func (txn *Txn) GetAll() []*Job
- func (txn *Txn) GetById(id string) *Job
- func (txn *Txn) GetByRunId(runId uuid.UUID) *Job
- func (txn *Txn) HasQueuedJobs(queue string) bool
- func (txn *Txn) QueuedJobs(queue string) *immutable.SortedSetIterator[*Job]
- func (txn *Txn) QueuedJobsByTtl() *immutable.SortedSetIterator[*Job]
- func (txn *Txn) UnvalidatedJobs() *immutable.SetIterator[*Job]
- func (txn *Txn) Upsert(jobs []*Job) error
- type UUIDHasher
- type UUIDProvider
Constants ¶
This section is empty.
Variables ¶
This section is empty.
Functions ¶
func SchedulingKeyFromJob ¶ added in v0.4.48
func SchedulingKeyFromJob(skg *schedulerobjects.SchedulingKeyGenerator, job *Job) schedulerobjects.SchedulingKey
func SchedulingOrderCompare ¶ added in v0.4.0
SchedulingOrderCompare defines the order in which jobs in a queue should be scheduled (both when scheduling new jobs and when re-scheduling evicted jobs). Specifically, compare returns
- 0 if the jobs have equal job id,
- -1 if job should be scheduled before other,
- +1 if other should be scheduled before other.
Types ¶
type Job ¶
type Job struct {
// contains filtered or unexported fields
}
Job is the scheduler-internal representation of a job.
func (*Job) ActiveRunTimestamp ¶ added in v0.4.12
ActiveRunTimestamp returns the creation time of the most recent run associated with this job.
func (*Job) Annotations ¶ added in v0.4.48
Annotations returns the annotations on the job.
func (*Job) Assert ¶ added in v0.4.12
Assert makes the assertions outlined below and returns - nil if the job is valid and - an error explaining why not otherwise.
Assertions: Required fields must be set.
All associated runs are valid. The state of LatestRun is compatible with that of the job.
The states {Queued, Running, Cancelled, Failed, Succeeded} are mutually exclusive.
Only these job state transitions are valid: - Queued -> {Running, Cancelled, Failed} - Running -> {Queued, Cancelled, Failed, Succeeded} - Succeeded, Cancelled, Failed -> {}
QueuedVersion is 0 initially and is incremented by 1 every time the queued status of the job changes. Hence: - If the job is queued, the queuedVersion must be 2x the number of runs. E.g.:
- queued (qv = 0)
- queued -> running -> queued: qv = 2
- If the job is running or succeeded, the queuedVersion must be one less than 2x number of runs. E.g.:
- queued -> running -> succeeded: qv = 1
- queued -> running -> queued -> running -> succeeded: qv = 3
- If the job is failed or cancelled, the queuedVersion may be either equal to or 1 less than 2x the number of runs. E.g.:
- queued -> failed/cancelled: qv = 0
- queued -> running -> failed/cancelled: qv = 1
- queued -> running -> queued -> failed/cancelled: qv = 2
func (*Job) CancelByJobsetRequested ¶
CancelByJobsetRequested returns true if the user has requested this job's jobSet be cancelled.
func (*Job) CancelRequested ¶
CancelRequested returns true if the user has requested this job be cancelled.
func (*Job) Equal ¶ added in v0.4.0
Equal returns true if job is equal to other and false otherwise. Scheduling requirements are assumed to be equal if both jobs have equal schedulingKey.
func (*Job) HasQueueTtlExpired ¶ added in v0.3.95
HasQueueTtlExpired returns true if the given job has reached its queueTtl expiry. Invariants:
- job.created < `t`
func (*Job) HasQueueTtlSet ¶ added in v0.3.95
HasQueueTtlSet returns true if the given job has a queueTtl set.
func (*Job) HasRuns ¶
HasRuns returns true if the job has been run If this is returns true then LatestRun is guaranteed to return a non-nil value.
func (*Job) InTerminalState ¶
InTerminalState returns true if the job is in a terminal state
func (*Job) JobSchedulingInfo ¶
func (job *Job) JobSchedulingInfo() *schedulerobjects.JobSchedulingInfo
JobSchedulingInfo returns the scheduling requirements associated with the job
func (*Job) LatestRun ¶
LatestRun returns the currently active job run or nil if there are no runs yet. Callers should either guard against nil values explicitly or via HasRuns.
func (*Job) NodeSelector ¶ added in v0.4.48
NodeSelector returns the Node Selector requested by the Job
func (*Job) NumAttempts ¶ added in v0.3.63
NumAttempts returns the number of times the executors tried to run this job. Note that this is O(N) on Runs, but this should be fine as the number of runs should be small.
func (*Job) NumReturned ¶
NumReturned returns the number of times this job has been returned by executors Note that this is O(N) on Runs, but this should be fine as the number of runs should be small.
func (*Job) PodRequirements ¶ added in v0.3.78
func (job *Job) PodRequirements() *schedulerobjects.PodRequirements
PodRequirements returns the pod requirements of the Job
func (*Job) PriorityClass ¶ added in v0.4.48
func (job *Job) PriorityClass() types.PriorityClass
PriorityClass returns the priority class of the job.
func (*Job) PriorityClassName ¶ added in v0.4.48
PriorityClassName returns the name of the job's Priority Class TODO: this can be inconsistent with job.PriorityClass()
func (*Job) QueueTtlSeconds ¶ added in v0.4.48
QueueTtlSeconds returns the time in seconds that the job should remain queued 0 means that this field is unset
func (*Job) Queued ¶
Queued returns true if the job should be considered by the scheduler for assignment or false otherwise.
func (*Job) QueuedVersion ¶ added in v0.3.63
QueuedVersion returns current queued state version.
func (*Job) RequestedPriority ¶
RequestedPriority returns the requested priority of the job.
func (*Job) ResourceRequirements ¶ added in v0.4.48
func (job *Job) ResourceRequirements() v1.ResourceRequirements
ResourceRequirements returns the resource requirements of the Job
func (*Job) RunById ¶
RunById returns the Run corresponding to the provided run id or nil if no such Run exists.
func (*Job) ScheduledAtPriority ¶ added in v0.4.48
ScheduledAtPriority returns the numeric priority at which the job was scheduled This will return false if the job has not been scheduled yet
func (*Job) SchedulingKey ¶ added in v0.4.48
func (job *Job) SchedulingKey() schedulerobjects.SchedulingKey
SchedulingKey returns the scheduling key associated with a job.
func (*Job) SchedulingOrderCompare ¶ added in v0.4.0
SchedulingOrderCompare defines the order in which jobs in a particular queue should be scheduled,
func (*Job) SubmitTime ¶ added in v0.4.48
SubmitTime exists for compatibility with the LegacyJob interface.
func (*Job) Tolerations ¶ added in v0.4.48
func (job *Job) Tolerations() []v1.Toleration
Tolerations returns the Tolerations requested by the Job
func (*Job) WithCancelByJobsetRequested ¶
WithCancelByJobsetRequested returns a copy of the job with the cancelByJobSetRequested status updated.
func (*Job) WithCancelRequested ¶
WithCancelRequested returns a copy of the job with the cancelRequested status updated.
func (*Job) WithCancelled ¶
WithCancelled returns a copy of the job with the cancelled status updated
func (*Job) WithCreated ¶ added in v0.3.53
WithCreated returns a copy of the job with the creation time updated.
func (*Job) WithFailed ¶
WithFailed returns a copy of the job with the failed status updated.
func (*Job) WithJobSchedulingInfo ¶ added in v0.3.54
func (job *Job) WithJobSchedulingInfo(jobSchedulingInfo *schedulerobjects.JobSchedulingInfo) *Job
WithJobSchedulingInfo returns a copy of the job with the job scheduling info updated.
func (*Job) WithJobset ¶ added in v0.3.54
WithJobset returns a copy of the job with the jobSet updated.
func (*Job) WithNewRun ¶
func (job *Job) WithNewRun(executor string, nodeId, nodeName string, scheduledAtPriority int32) *Job
WithNewRun creates a copy of the job with a new run on the given executor.
func (*Job) WithPriority ¶
WithPriority returns a copy of the job with the priority updated.
func (*Job) WithPriorityClass ¶ added in v0.4.48
func (job *Job) WithPriorityClass(priorityClass types.PriorityClass) *Job
WithPriorityClass returns a copy of the job with the priority class updated.
func (*Job) WithQueue ¶ added in v0.3.54
WithQueue returns a copy of the job with the queue updated.
func (*Job) WithQueued ¶
WithQueued returns a copy of the job with the queued status updated.
func (*Job) WithQueuedVersion ¶ added in v0.3.63
WithQueuedVersion returns a copy of the job with the queued version updated.
func (*Job) WithRequestedPriority ¶
WithRequestedPriority returns a copy of the job with the priority updated.
func (*Job) WithSubmittedTime ¶ added in v0.4.12
WithSubmittedTime returns a copy of the job with submittedTime updated.
func (*Job) WithSucceeded ¶
WithSucceeded returns a copy of the job with the succeeded status updated.
func (*Job) WithUpdatedRun ¶
WithUpdatedRun returns a copy of the job with the provided run upserted.
func (*Job) WithValidated ¶ added in v0.4.50
WithValidated returns a copy of the job with the validated updated.
func (*Job) WithoutTerminal ¶ added in v0.4.35
type JobDb ¶
type JobDb struct {
// contains filtered or unexported fields
}
func NewJobDb ¶
func NewJobDb(priorityClasses map[string]types.PriorityClass, defaultPriorityClassName string, stringInterner *stringinterner.StringInterner) *JobDb
func NewJobDbWithSchedulingKeyGenerator ¶ added in v0.3.100
func NewJobDbWithSchedulingKeyGenerator( priorityClasses map[string]types.PriorityClass, defaultPriorityClassName string, skg *schedulerobjects.SchedulingKeyGenerator, stringInterner *stringinterner.StringInterner, ) *JobDb
func (*JobDb) CreateRun ¶ added in v0.4.5
func (jobDb *JobDb) CreateRun( id uuid.UUID, jobId string, creationTime int64, executor string, nodeId string, nodeName string, scheduledAtPriority *int32, leased bool, pending bool, running bool, preemptRequested bool, preempted bool, succeeded bool, failed bool, cancelled bool, leaseTime *time.Time, pendingTime *time.Time, runningTime *time.Time, preemptedTime *time.Time, terminatedTime *time.Time, returned bool, runAttempted bool, ) *JobRun
CreateRun creates a new scheduler job run from a database job run
func (*JobDb) NewJob ¶ added in v0.3.100
func (jobDb *JobDb) NewJob( jobId string, jobSet string, queue string, priority uint32, schedulingInfo *schedulerobjects.JobSchedulingInfo, queued bool, queuedVersion int32, cancelRequested bool, cancelByJobSetRequested bool, cancelled bool, created int64, validated bool, ) *Job
NewJob creates a new scheduler job. The new job is not automatically inserted into the jobDb; call jobDb.Upsert to upsert it.
func (*JobDb) ReadTxn ¶
ReadTxn returns a read-only transaction. Multiple read-only transactions can access the db concurrently
func (*JobDb) ReconcileDifferences ¶ added in v0.4.5
func (jobDb *JobDb) ReconcileDifferences(txn *Txn, jobRepoJobs []database.Job, jobRepoRuns []database.Run) ([]JobStateTransitions, error)
ReconcileDifferences reconciles any differences between jobs stored in the jobDb with those provided to this function and returns the updated jobs together with a summary of the state transitions applied to those jobs.
func (*JobDb) SetClock ¶ added in v0.4.12
func (jobDb *JobDb) SetClock(clock clock.PassiveClock)
func (*JobDb) SetUUIDProvider ¶ added in v0.4.12
func (jobDb *JobDb) SetUUIDProvider(uuidProvider UUIDProvider)
type JobIdHasher ¶ added in v0.4.50
type JobIdHasher struct{}
func (JobIdHasher) Equal ¶ added in v0.4.50
func (JobIdHasher) Equal(a, b *Job) bool
func (JobIdHasher) Hash ¶ added in v0.4.50
func (JobIdHasher) Hash(j *Job) uint32
type JobPriorityComparer ¶ added in v0.3.53
type JobPriorityComparer struct{}
func (JobPriorityComparer) Compare ¶ added in v0.3.53
func (JobPriorityComparer) Compare(job, other *Job) int
type JobQueueTtlComparer ¶ added in v0.3.95
type JobQueueTtlComparer struct{}
func (JobQueueTtlComparer) Compare ¶ added in v0.3.95
func (j JobQueueTtlComparer) Compare(a, b *Job) int
Compare jobs by their remaining queue time before expiry Invariants:
- Job.queueTtl must be > 0
- Job.created must be < `t`
type JobRun ¶
type JobRun struct {
// contains filtered or unexported fields
}
JobRun is the scheduler-internal representation of a job run.
There are columns in the `runs` table that are not needed in the scheduler, such as `pod_requirements_overlay`; these are not represented here.
func (*JobRun) Assert ¶ added in v0.4.12
Assert makes the assertions outlined below and returns - nil if the run is valid and - an error explaining why not otherwise.
Assertions: Required fields must be set.
The states {Running, Cancelled, Failed, Succeeded, Returned} are mutually exclusive.
func (*JobRun) InTerminalState ¶
InTerminalState returns true if the JobRun is in a terminal state
func (*JobRun) JobId ¶ added in v0.3.54
JobId returns the id of the job this run is associated with.
func (*JobRun) NodeId ¶ added in v0.3.82
NodeId returns the id of the node to which the JobRun is assigned.
func (*JobRun) NodeName ¶ added in v0.3.82
NodeName returns the name of the node to which the JobRun is assigned.
func (*JobRun) PendingTime ¶ added in v0.4.19
func (*JobRun) PreemptRequested ¶ added in v0.4.41
PreemptRequested Returns true if there has been a request this run is preempted
func (*JobRun) Preempted ¶ added in v0.4.19
Preempted Returns true if the executor has reported the job run as preempted
func (*JobRun) PreemptedTime ¶ added in v0.4.19
func (*JobRun) RunAttempted ¶ added in v0.3.63
RunAttempted Returns true if the executor has attempted to run the job.
func (*JobRun) RunningTime ¶ added in v0.4.19
func (*JobRun) ScheduledAtPriority ¶ added in v0.4.8
func (*JobRun) Succeeded ¶
Succeeded Returns true if the executor has reported the job run as successful
func (*JobRun) TerminatedTime ¶ added in v0.4.19
func (*JobRun) WithAttempted ¶ added in v0.3.63
WithAttempted returns a copy of the job run with the runAttempted status updated.
func (*JobRun) WithCancelled ¶
WithCancelled returns a copy of the job run with the cancelled status updated.
func (*JobRun) WithFailed ¶
WithFailed returns a copy of the job run with the failed status updated.
func (*JobRun) WithLeased ¶ added in v0.4.19
func (*JobRun) WithLeasedTime ¶ added in v0.4.19
func (*JobRun) WithPending ¶ added in v0.4.19
func (*JobRun) WithPendingTime ¶ added in v0.4.19
func (*JobRun) WithPreemptRequested ¶ added in v0.4.41
WithPreemptRequested returns a copy of the job run with the preemptRequested status updated.
func (*JobRun) WithPreempted ¶ added in v0.4.19
WithPreempted returns a copy of the job run with the preempted status updated.
func (*JobRun) WithPreemptedTime ¶ added in v0.4.19
func (*JobRun) WithReturned ¶
func (*JobRun) WithRunning ¶
WithRunning returns a copy of the job run with the running status updated.
func (*JobRun) WithRunningTime ¶ added in v0.4.19
func (*JobRun) WithSucceeded ¶
WithSucceeded returns a copy of the job run with the succeeded status updated.
func (*JobRun) WithTerminatedTime ¶ added in v0.4.19
func (*JobRun) WithoutTerminal ¶ added in v0.4.20
WithoutTerminal returns a copy of the job run with the terminal states set to false to prevent conflicts. terminal states are {succeeded, failed, cancelled, preempted}.
type JobStateTransitions ¶ added in v0.4.5
type JobStateTransitions struct { Job *Job Queued bool Leased bool Pending bool Running bool Cancelled bool Preempted bool Failed bool Succeeded bool }
JobStateTransitions captures the process of updating a job. It bundles the updated job with booleans indicating which state transitions were applied to produce it. These are cumulative in the sense that a job with transitions queued -> scheduled -> queued -> running -> failed will have the fields queued, scheduled, running, and failed set to true.
type RealUUIDProvider ¶ added in v0.4.12
type RealUUIDProvider struct{}
RealUUIDProvider calls uuid.New.
func (RealUUIDProvider) New ¶ added in v0.4.12
func (_ RealUUIDProvider) New() uuid.UUID
type RunStateTransitions ¶ added in v0.4.5
type RunStateTransitions struct { JobRun *JobRun Leased bool Returned bool Pending bool Running bool Cancelled bool Preempted bool Failed bool Succeeded bool }
RunStateTransitions captures the process of updating a run. It works in the same way as JobStateTransitions does for jobs.
type Txn ¶ added in v0.3.53
type Txn struct {
// contains filtered or unexported fields
}
Txn is a JobDb Transaction. Transactions provide a consistent view of the database, allowing readers to perform multiple actions without the database changing from underneath them. Write transactions also allow callers to perform write operations that will not be visible to other users until the transaction is committed.
func (*Txn) Assert ¶ added in v0.4.12
Assert returns an error if the jobDb, or any job stored in the jobDb, is in an invalid state. If assertOnlyActiveJobs is true, it also asserts that all jobs in the jobDb are active.
func (*Txn) AssertEqual ¶ added in v0.4.12
func (*Txn) BatchDelete ¶ added in v0.4.1
BatchDelete deletes the jobs with the given ids from the database. Any ids not in the database are ignored.
func (*Txn) GetById ¶ added in v0.4.1
GetById returns the job with the given Id or nil if no such job exists The Job returned by this function *must not* be subsequently modified
func (*Txn) GetByRunId ¶ added in v0.4.1
GetByRunId returns the job with the given run id or nil if no such job exists The Job returned by this function *must not* be subsequently modified
func (*Txn) HasQueuedJobs ¶ added in v0.4.1
HasQueuedJobs returns true if the queue has any jobs in the running state or false otherwise
func (*Txn) QueuedJobs ¶ added in v0.4.1
func (txn *Txn) QueuedJobs(queue string) *immutable.SortedSetIterator[*Job]
QueuedJobs returns true if the queue has any jobs in the running state or false otherwise
func (*Txn) QueuedJobsByTtl ¶ added in v0.4.1
func (txn *Txn) QueuedJobsByTtl() *immutable.SortedSetIterator[*Job]
QueuedJobsByTtl returns an iterator for jobs ordered by queue ttl time - the closest to expiry first
func (*Txn) UnvalidatedJobs ¶ added in v0.4.50
func (txn *Txn) UnvalidatedJobs() *immutable.SetIterator[*Job]
UnvalidatedJobs returns an iterator for jobs ordered by queue ttl time - the closest to expiry first
type UUIDHasher ¶ added in v0.3.89
type UUIDHasher struct{}
UUIDHasher is an implementation of Hasher for UUID.
type UUIDProvider ¶ added in v0.4.12
UUIDProvider is an interface used to mock UUID generation for tests.