open-nomad/nomad/state/state_store.go

1473 lines
39 KiB
Go
Raw Normal View History

package state
2015-06-01 15:49:10 +00:00
import (
"fmt"
2015-06-01 15:49:10 +00:00
"io"
"log"
"sync"
"github.com/hashicorp/go-memdb"
2015-07-04 01:19:43 +00:00
"github.com/hashicorp/nomad/nomad/structs"
"github.com/hashicorp/nomad/nomad/watch"
2015-06-01 15:49:10 +00:00
)
// IndexEntry is used with the "index" table
// for managing the latest Raft index affecting a table.
type IndexEntry struct {
Key string
Value uint64
}
2015-07-04 00:50:54 +00:00
// The StateStore is responsible for maintaining all the Nomad
2015-06-01 15:49:10 +00:00
// state. It is manipulated by the FSM which maintains consistency
// through the use of Raft. The goals of the StateStore are to provide
// high concurrency for read operations without blocking writes, and
2015-07-04 01:19:43 +00:00
// to provide write availability in the face of reads. EVERY object
// returned as a result of a read against the state store should be
// considered a constant and NEVER modified in place.
2015-06-01 15:49:10 +00:00
type StateStore struct {
logger *log.Logger
db *memdb.MemDB
watch *stateWatch
2015-06-01 15:49:10 +00:00
}
// NewStateStore is used to create a new state store
func NewStateStore(logOutput io.Writer) (*StateStore, error) {
// Create the MemDB
db, err := memdb.NewMemDB(stateStoreSchema())
if err != nil {
return nil, fmt.Errorf("state store setup failed: %v", err)
}
// Create the state store
2015-06-01 15:49:10 +00:00
s := &StateStore{
logger: log.New(logOutput, "", log.LstdFlags),
db: db,
watch: newStateWatch(),
2015-06-01 15:49:10 +00:00
}
return s, nil
}
// Snapshot is used to create a point in time snapshot. Because
// we use MemDB, we just need to snapshot the state of the underlying
// database.
2015-06-01 15:49:10 +00:00
func (s *StateStore) Snapshot() (*StateSnapshot, error) {
snap := &StateSnapshot{
StateStore: StateStore{
logger: s.logger,
db: s.db.Snapshot(),
watch: s.watch,
},
}
return snap, nil
2015-06-01 15:49:10 +00:00
}
2015-07-04 01:19:43 +00:00
// Restore is used to optimize the efficiency of rebuilding
// state by minimizing the number of transactions and checking
// overhead.
func (s *StateStore) Restore() (*StateRestore, error) {
txn := s.db.Txn(true)
r := &StateRestore{
txn: txn,
watch: s.watch,
items: watch.NewItems(),
}
return r, nil
}
// Watch subscribes a channel to a set of watch items.
func (s *StateStore) Watch(items watch.Items, notify chan struct{}) {
s.watch.watch(items, notify)
}
// StopWatch unsubscribes a channel from a set of watch items.
func (s *StateStore) StopWatch(items watch.Items, notify chan struct{}) {
s.watch.stopWatch(items, notify)
}
// UpsertNode is used to register a node or update a node definition
2015-09-07 02:51:50 +00:00
// This is assumed to be triggered by the client, so we retain the value
// of drain which is set by the scheduler.
func (s *StateStore) UpsertNode(index uint64, node *structs.Node) error {
2015-07-04 01:19:43 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "nodes"})
watcher.Add(watch.Item{Node: node.ID})
2015-07-04 01:19:43 +00:00
// Check if the node already exists
existing, err := txn.First("nodes", "id", node.ID)
if err != nil {
return fmt.Errorf("node lookup failed: %v", err)
}
// Setup the indexes correctly
if existing != nil {
2015-09-07 02:51:50 +00:00
exist := existing.(*structs.Node)
node.CreateIndex = exist.CreateIndex
2015-07-04 01:19:43 +00:00
node.ModifyIndex = index
2015-09-07 02:51:50 +00:00
node.Drain = exist.Drain // Retain the drain mode
2015-07-04 01:19:43 +00:00
} else {
node.CreateIndex = index
node.ModifyIndex = index
}
// Insert the node
if err := txn.Insert("nodes", node); err != nil {
return fmt.Errorf("node insert failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"nodes", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
2015-07-04 01:19:43 +00:00
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-04 01:19:43 +00:00
txn.Commit()
return nil
}
// DeleteNode is used to deregister a node
func (s *StateStore) DeleteNode(index uint64, nodeID string) error {
2015-07-04 01:19:43 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
// Lookup the node
existing, err := txn.First("nodes", "id", nodeID)
if err != nil {
return fmt.Errorf("node lookup failed: %v", err)
}
if existing == nil {
return fmt.Errorf("node not found")
}
2015-12-16 21:46:09 +00:00
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "nodes"})
watcher.Add(watch.Item{Node: nodeID})
2015-07-04 01:19:43 +00:00
// Delete the node
if err := txn.Delete("nodes", existing); err != nil {
return fmt.Errorf("node delete failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"nodes", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
2015-07-04 01:19:43 +00:00
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-04 01:19:43 +00:00
txn.Commit()
return nil
}
// UpdateNodeStatus is used to update the status of a node
2015-09-07 02:51:50 +00:00
func (s *StateStore) UpdateNodeStatus(index uint64, nodeID, status string) error {
2015-07-04 01:19:43 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "nodes"})
watcher.Add(watch.Item{Node: nodeID})
2015-07-04 01:19:43 +00:00
// Lookup the node
existing, err := txn.First("nodes", "id", nodeID)
if err != nil {
return fmt.Errorf("node lookup failed: %v", err)
}
if existing == nil {
return fmt.Errorf("node not found")
}
// Copy the existing node
existingNode := existing.(*structs.Node)
copyNode := new(structs.Node)
*copyNode = *existingNode
// Update the status in the copy
copyNode.Status = status
copyNode.ModifyIndex = index
// Insert the node
if err := txn.Insert("nodes", copyNode); err != nil {
return fmt.Errorf("node update failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"nodes", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
2015-07-04 01:19:43 +00:00
// Update the state of the allocations which are in running state to lost
if status == structs.NodeStatusDown {
allocs, err := s.AllocsByNode(nodeID)
if err != nil {
return fmt.Errorf("error retrieving any allocations for the node: %v", nodeID)
}
for _, alloc := range allocs {
copyAlloc := new(structs.Allocation)
*copyAlloc = *alloc
if alloc.ClientStatus == structs.AllocClientStatusPending ||
alloc.ClientStatus == structs.AllocClientStatusRunning {
copyAlloc.ClientStatus = structs.AllocClientStatusLost
if err := txn.Insert("allocs", copyAlloc); err != nil {
return fmt.Errorf("alloc insert failed: %v", err)
}
}
}
}
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-04 01:19:43 +00:00
txn.Commit()
return nil
}
2015-09-07 02:51:50 +00:00
// UpdateNodeDrain is used to update the drain of a node
func (s *StateStore) UpdateNodeDrain(index uint64, nodeID string, drain bool) error {
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "nodes"})
watcher.Add(watch.Item{Node: nodeID})
2015-09-07 02:51:50 +00:00
// Lookup the node
existing, err := txn.First("nodes", "id", nodeID)
if err != nil {
return fmt.Errorf("node lookup failed: %v", err)
}
if existing == nil {
return fmt.Errorf("node not found")
}
// Copy the existing node
existingNode := existing.(*structs.Node)
copyNode := new(structs.Node)
*copyNode = *existingNode
// Update the drain in the copy
copyNode.Drain = drain
copyNode.ModifyIndex = index
// Insert the node
if err := txn.Insert("nodes", copyNode); err != nil {
return fmt.Errorf("node update failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"nodes", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
2015-09-07 02:51:50 +00:00
txn.Commit()
return nil
}
2015-09-07 03:51:01 +00:00
// NodeByID is used to lookup a node by ID
func (s *StateStore) NodeByID(nodeID string) (*structs.Node, error) {
2015-07-04 01:19:43 +00:00
txn := s.db.Txn(false)
existing, err := txn.First("nodes", "id", nodeID)
2015-07-04 01:19:43 +00:00
if err != nil {
return nil, fmt.Errorf("node lookup failed: %v", err)
}
if existing != nil {
return existing.(*structs.Node), nil
2015-07-04 01:19:43 +00:00
}
return nil, nil
}
// NodesByIDPrefix is used to lookup nodes by prefix
func (s *StateStore) NodesByIDPrefix(nodeID string) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("nodes", "id_prefix", nodeID)
if err != nil {
return nil, fmt.Errorf("node lookup failed: %v", err)
}
return iter, nil
}
// Nodes returns an iterator over all the nodes
func (s *StateStore) Nodes() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Walk the entire nodes table
iter, err := txn.Get("nodes", "id")
if err != nil {
return nil, err
}
return iter, nil
}
// UpsertJob is used to register a job or update a job definition
func (s *StateStore) UpsertJob(index uint64, job *structs.Job) error {
2015-07-07 16:41:05 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "jobs"})
watcher.Add(watch.Item{Job: job.ID})
2015-07-07 16:41:05 +00:00
// Check if the job already exists
existing, err := txn.First("jobs", "id", job.ID)
2015-07-07 16:41:05 +00:00
if err != nil {
return fmt.Errorf("job lookup failed: %v", err)
}
// Setup the indexes correctly
if existing != nil {
job.CreateIndex = existing.(*structs.Job).CreateIndex
job.ModifyIndex = index
2016-01-12 17:50:33 +00:00
job.JobModifyIndex = index
2016-01-12 01:34:25 +00:00
// Compute the job status
var err error
job.Status, err = s.getJobStatus(txn, job, false)
if err != nil {
return fmt.Errorf("setting job status for %q failed: %v", job.ID, err)
}
2015-07-07 16:41:05 +00:00
} else {
job.CreateIndex = index
job.ModifyIndex = index
2016-01-12 17:50:33 +00:00
job.JobModifyIndex = index
2016-01-09 02:22:59 +00:00
2016-01-12 01:34:25 +00:00
// If we are inserting the job for the first time, we don't need to
// calculate the jobs status as it is known.
2016-01-09 02:22:59 +00:00
if job.IsPeriodic() {
2016-01-12 01:34:25 +00:00
job.Status = structs.JobStatusRunning
2016-01-09 02:22:59 +00:00
} else {
2016-01-12 01:34:25 +00:00
job.Status = structs.JobStatusPending
2016-01-09 02:22:59 +00:00
}
}
2015-07-07 16:41:05 +00:00
// Insert the job
if err := txn.Insert("jobs", job); err != nil {
return fmt.Errorf("job insert failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"jobs", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
// Update the job summary
if err := s.updateSummaryWithJob(job, txn); err != nil {
return fmt.Errorf("job summary update failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-07 16:41:05 +00:00
txn.Commit()
return nil
}
// DeleteJob is used to deregister a job
func (s *StateStore) DeleteJob(index uint64, jobID string) error {
2015-07-07 16:41:05 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
// Lookup the node
existing, err := txn.First("jobs", "id", jobID)
2015-07-07 16:41:05 +00:00
if err != nil {
return fmt.Errorf("job lookup failed: %v", err)
2015-07-07 16:41:05 +00:00
}
if existing == nil {
return fmt.Errorf("job not found")
}
2015-12-16 21:46:09 +00:00
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "jobs"})
watcher.Add(watch.Item{Job: jobID})
2015-07-07 16:41:05 +00:00
// Delete the node
if err := txn.Delete("jobs", existing); err != nil {
return fmt.Errorf("job delete failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"jobs", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
// Delete the job summary
if _, err = txn.DeleteAll("jobsummary", "id", jobID); err != nil {
return fmt.Errorf("deleing job summary failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-07 16:41:05 +00:00
txn.Commit()
return nil
}
2015-09-07 03:51:01 +00:00
// JobByID is used to lookup a job by its ID
func (s *StateStore) JobByID(id string) (*structs.Job, error) {
2015-07-07 16:41:05 +00:00
txn := s.db.Txn(false)
existing, err := txn.First("jobs", "id", id)
2015-07-07 16:41:05 +00:00
if err != nil {
return nil, fmt.Errorf("job lookup failed: %v", err)
}
if existing != nil {
return existing.(*structs.Job), nil
2015-07-07 16:41:05 +00:00
}
return nil, nil
}
// JobsByIDPrefix is used to lookup a job by prefix
func (s *StateStore) JobsByIDPrefix(id string) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("jobs", "id_prefix", id)
if err != nil {
return nil, fmt.Errorf("job lookup failed: %v", err)
}
return iter, nil
}
2015-07-07 16:41:05 +00:00
// Jobs returns an iterator over all the jobs
func (s *StateStore) Jobs() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Walk the entire jobs table
iter, err := txn.Get("jobs", "id")
if err != nil {
return nil, err
}
return iter, nil
}
2015-12-04 17:49:42 +00:00
// JobsByPeriodic returns an iterator over all the periodic or non-periodic jobs.
func (s *StateStore) JobsByPeriodic(periodic bool) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("jobs", "periodic", periodic)
if err != nil {
return nil, err
}
return iter, nil
}
// JobsByScheduler returns an iterator over all the jobs with the specific
// scheduler type.
func (s *StateStore) JobsByScheduler(schedulerType string) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Return an iterator for jobs with the specific type.
iter, err := txn.Get("jobs", "type", schedulerType)
if err != nil {
return nil, err
}
return iter, nil
}
2015-12-15 03:20:57 +00:00
// JobsByGC returns an iterator over all jobs eligible or uneligible for garbage
// collection.
func (s *StateStore) JobsByGC(gc bool) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("jobs", "gc", gc)
if err != nil {
return nil, err
}
return iter, nil
}
// JobSummary returns a job summary object which matches a specific id.
func (s *StateStore) JobSummaryByID(jobID string) (*structs.JobSummary, error) {
txn := s.db.Txn(false)
existing, err := txn.First("jobsummary", "id", jobID)
if err != nil {
return nil, err
}
if existing != nil {
return existing.(*structs.JobSummary), nil
}
return nil, nil
}
// JobSummaries walks the entire job summary table and retuns all the job
// summary objects
func (s *StateStore) JobSummaries() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("jobsummary", "id")
if err != nil {
return nil, err
}
return iter, nil
}
2015-12-19 01:51:30 +00:00
// UpsertPeriodicLaunch is used to register a launch or update it.
func (s *StateStore) UpsertPeriodicLaunch(index uint64, launch *structs.PeriodicLaunch) error {
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "periodic_launch"})
watcher.Add(watch.Item{Job: launch.ID})
// Check if the job already exists
2015-12-16 21:46:09 +00:00
existing, err := txn.First("periodic_launch", "id", launch.ID)
if err != nil {
2015-12-19 01:51:30 +00:00
return fmt.Errorf("periodic launch lookup failed: %v", err)
}
2015-12-16 21:46:09 +00:00
// Setup the indexes correctly
if existing != nil {
launch.CreateIndex = existing.(*structs.PeriodicLaunch).CreateIndex
launch.ModifyIndex = index
} else {
launch.CreateIndex = index
launch.ModifyIndex = index
}
2015-12-19 01:51:30 +00:00
// Insert the job
if err := txn.Insert("periodic_launch", launch); err != nil {
return fmt.Errorf("launch insert failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"periodic_launch", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
txn.Commit()
return nil
}
// DeletePeriodicLaunch is used to delete the periodic launch
func (s *StateStore) DeletePeriodicLaunch(index uint64, jobID string) error {
txn := s.db.Txn(true)
defer txn.Abort()
// Lookup the launch
existing, err := txn.First("periodic_launch", "id", jobID)
if err != nil {
return fmt.Errorf("launch lookup failed: %v", err)
}
if existing == nil {
return fmt.Errorf("launch not found")
}
2015-12-16 21:46:09 +00:00
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "periodic_launch"})
watcher.Add(watch.Item{Job: jobID})
2015-12-19 01:51:30 +00:00
// Delete the launch
if err := txn.Delete("periodic_launch", existing); err != nil {
return fmt.Errorf("launch delete failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"periodic_launch", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
txn.Commit()
return nil
}
// PeriodicLaunchByID is used to lookup a periodic launch by the periodic job
// ID.
func (s *StateStore) PeriodicLaunchByID(id string) (*structs.PeriodicLaunch, error) {
txn := s.db.Txn(false)
existing, err := txn.First("periodic_launch", "id", id)
if err != nil {
return nil, fmt.Errorf("periodic launch lookup failed: %v", err)
}
if existing != nil {
return existing.(*structs.PeriodicLaunch), nil
}
return nil, nil
}
2015-12-07 23:58:17 +00:00
// PeriodicLaunches returns an iterator over all the periodic launches
func (s *StateStore) PeriodicLaunches() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Walk the entire table
iter, err := txn.Get("periodic_launch", "id")
if err != nil {
return nil, err
}
return iter, nil
}
2015-07-23 22:43:06 +00:00
// UpsertEvaluation is used to upsert an evaluation
func (s *StateStore) UpsertEvals(index uint64, evals []*structs.Evaluation) error {
2015-07-23 22:43:06 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "evals"})
2015-07-23 22:43:06 +00:00
// Do a nested upsert
2016-01-09 02:22:59 +00:00
jobs := make(map[string]string, len(evals))
for _, eval := range evals {
watcher.Add(watch.Item{Eval: eval.ID})
if err := s.nestedUpsertEval(txn, index, eval); err != nil {
return err
}
2016-01-09 02:22:59 +00:00
jobs[eval.JobID] = ""
}
// Set the job's status
2016-01-12 01:34:25 +00:00
if err := s.setJobStatuses(index, watcher, txn, jobs, false); err != nil {
2016-01-09 02:22:59 +00:00
return fmt.Errorf("setting job status failed: %v", err)
2015-07-23 22:43:06 +00:00
}
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-23 22:43:06 +00:00
txn.Commit()
return nil
}
// nestedUpsertEvaluation is used to nest an evaluation upsert within a transaction
func (s *StateStore) nestedUpsertEval(txn *memdb.Txn, index uint64, eval *structs.Evaluation) error {
// Lookup the evaluation
existing, err := txn.First("evals", "id", eval.ID)
if err != nil {
return fmt.Errorf("eval lookup failed: %v", err)
}
// Update the indexes
if existing != nil {
eval.CreateIndex = existing.(*structs.Evaluation).CreateIndex
eval.ModifyIndex = index
} else {
eval.CreateIndex = index
eval.ModifyIndex = index
}
// Insert the eval
if err := txn.Insert("evals", eval); err != nil {
return fmt.Errorf("eval insert failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"evals", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
return nil
}
// DeleteEval is used to delete an evaluation
func (s *StateStore) DeleteEval(index uint64, evals []string, allocs []string) error {
2015-07-23 22:43:06 +00:00
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "evals"})
watcher.Add(watch.Item{Table: "allocs"})
2015-07-23 22:43:06 +00:00
2016-01-09 02:22:59 +00:00
jobs := make(map[string]string, len(evals))
for _, eval := range evals {
existing, err := txn.First("evals", "id", eval)
if err != nil {
return fmt.Errorf("eval lookup failed: %v", err)
}
if existing == nil {
continue
}
if err := txn.Delete("evals", existing); err != nil {
return fmt.Errorf("eval delete failed: %v", err)
}
watcher.Add(watch.Item{Eval: eval})
2016-01-09 02:22:59 +00:00
jobs[existing.(*structs.Evaluation).JobID] = ""
2015-07-23 22:43:06 +00:00
}
for _, alloc := range allocs {
existing, err := txn.First("allocs", "id", alloc)
if err != nil {
return fmt.Errorf("alloc lookup failed: %v", err)
}
if existing == nil {
continue
}
if err := txn.Delete("allocs", existing); err != nil {
return fmt.Errorf("alloc delete failed: %v", err)
}
2015-10-30 15:27:47 +00:00
realAlloc := existing.(*structs.Allocation)
watcher.Add(watch.Item{Alloc: realAlloc.ID})
watcher.Add(watch.Item{AllocEval: realAlloc.EvalID})
watcher.Add(watch.Item{AllocJob: realAlloc.JobID})
watcher.Add(watch.Item{AllocNode: realAlloc.NodeID})
2015-07-23 22:43:06 +00:00
}
// Update the indexes
2015-07-23 22:43:06 +00:00
if err := txn.Insert("index", &IndexEntry{"evals", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"allocs", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
2016-01-09 02:22:59 +00:00
// Set the job's status
2016-01-12 01:34:25 +00:00
if err := s.setJobStatuses(index, watcher, txn, jobs, true); err != nil {
2016-01-09 02:22:59 +00:00
return fmt.Errorf("setting job status failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
2015-07-23 22:43:06 +00:00
txn.Commit()
return nil
}
2015-09-07 03:51:01 +00:00
// EvalByID is used to lookup an eval by its ID
func (s *StateStore) EvalByID(id string) (*structs.Evaluation, error) {
2015-07-23 22:43:06 +00:00
txn := s.db.Txn(false)
existing, err := txn.First("evals", "id", id)
2015-07-23 22:43:06 +00:00
if err != nil {
return nil, fmt.Errorf("eval lookup failed: %v", err)
}
if existing != nil {
return existing.(*structs.Evaluation), nil
2015-07-23 22:43:06 +00:00
}
return nil, nil
}
// EvalsByIDPrefix is used to lookup evaluations by prefix
func (s *StateStore) EvalsByIDPrefix(id string) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("evals", "id_prefix", id)
if err != nil {
return nil, fmt.Errorf("eval lookup failed: %v", err)
}
return iter, nil
}
2015-09-06 19:10:24 +00:00
// EvalsByJob returns all the evaluations by job id
func (s *StateStore) EvalsByJob(jobID string) ([]*structs.Evaluation, error) {
txn := s.db.Txn(false)
// Get an iterator over the node allocations
iter, err := txn.Get("evals", "job", jobID)
if err != nil {
return nil, err
}
var out []*structs.Evaluation
for {
raw := iter.Next()
if raw == nil {
break
}
out = append(out, raw.(*structs.Evaluation))
}
return out, nil
}
2015-07-23 22:43:06 +00:00
// Evals returns an iterator over all the evaluations
func (s *StateStore) Evals() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Walk the entire table
iter, err := txn.Get("evals", "id")
if err != nil {
return nil, err
}
return iter, nil
}
// UpdateAllocFromClient is used to update an allocation based on input
// from a client. While the schedulers are the authority on the allocation for
// most things, some updates are authoritative from the client. Specifically,
// the desired state comes from the schedulers, while the actual state comes
// from clients.
func (s *StateStore) UpdateAllocsFromClient(index uint64, allocs []*structs.Allocation) error {
txn := s.db.Txn(true)
defer txn.Abort()
// Setup the watcher
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "allocs"})
// Handle each of the updated allocations
for _, alloc := range allocs {
if err := s.nestedUpdateAllocFromClient(txn, watcher, index, alloc); err != nil {
return err
}
}
// Update the indexes
if err := txn.Insert("index", &IndexEntry{"allocs", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
txn.Commit()
return nil
}
// nestedUpdateAllocFromClient is used to nest an update of an allocation with client status
func (s *StateStore) nestedUpdateAllocFromClient(txn *memdb.Txn, watcher watch.Items, index uint64, alloc *structs.Allocation) error {
// Look for existing alloc
existing, err := txn.First("allocs", "id", alloc.ID)
if err != nil {
return fmt.Errorf("alloc lookup failed: %v", err)
}
// Nothing to do if this does not exist
if existing == nil {
return nil
}
exist := existing.(*structs.Allocation)
// Update the job summary
if err := s.updateSummaryWithAlloc(alloc, exist, txn); err != nil {
return fmt.Errorf("unable to update job summary: %v", err)
}
// Trigger the watcher
watcher.Add(watch.Item{Alloc: alloc.ID})
watcher.Add(watch.Item{AllocEval: exist.EvalID})
watcher.Add(watch.Item{AllocJob: exist.JobID})
watcher.Add(watch.Item{AllocNode: exist.NodeID})
// Copy everything from the existing allocation
copyAlloc := new(structs.Allocation)
*copyAlloc = *exist
// Pull in anything the client is the authority on
copyAlloc.ClientStatus = alloc.ClientStatus
copyAlloc.ClientDescription = alloc.ClientDescription
copyAlloc.TaskStates = alloc.TaskStates
// Update the modify index
copyAlloc.ModifyIndex = index
// Update the allocation
if err := txn.Insert("allocs", copyAlloc); err != nil {
return fmt.Errorf("alloc insert failed: %v", err)
}
2016-01-09 02:22:59 +00:00
// Set the job's status
forceStatus := ""
if !copyAlloc.TerminalStatus() {
forceStatus = structs.JobStatusRunning
}
jobs := map[string]string{exist.JobID: forceStatus}
2016-01-12 01:34:25 +00:00
if err := s.setJobStatuses(index, watcher, txn, jobs, false); err != nil {
2016-01-09 02:22:59 +00:00
return fmt.Errorf("setting job status failed: %v", err)
}
return nil
}
// UpsertAllocs is used to evict a set of allocations
// and allocate new ones at the same time.
func (s *StateStore) UpsertAllocs(index uint64, allocs []*structs.Allocation) error {
txn := s.db.Txn(true)
defer txn.Abort()
watcher := watch.NewItems()
watcher.Add(watch.Item{Table: "allocs"})
// Handle the allocations
2016-01-12 01:34:25 +00:00
jobs := make(map[string]string, 1)
for _, alloc := range allocs {
existing, err := txn.First("allocs", "id", alloc.ID)
if err != nil {
return fmt.Errorf("alloc lookup failed: %v", err)
}
2015-08-25 23:26:34 +00:00
exist, _ := existing.(*structs.Allocation)
if err := s.updateSummaryWithAlloc(alloc, exist, txn); err != nil {
return fmt.Errorf("updating job summary failed: %v", err)
}
if exist == nil {
alloc.CreateIndex = index
alloc.ModifyIndex = index
alloc.AllocModifyIndex = index
} else {
2015-08-25 23:26:34 +00:00
alloc.CreateIndex = exist.CreateIndex
alloc.ModifyIndex = index
alloc.AllocModifyIndex = index
2015-08-25 23:26:34 +00:00
alloc.ClientStatus = exist.ClientStatus
alloc.ClientDescription = exist.ClientDescription
}
if err := txn.Insert("allocs", alloc); err != nil {
return fmt.Errorf("alloc insert failed: %v", err)
}
2016-01-12 01:34:25 +00:00
// If the allocation is running, force the job to running status.
2016-01-09 02:22:59 +00:00
forceStatus := ""
if !alloc.TerminalStatus() {
forceStatus = structs.JobStatusRunning
}
jobs[alloc.JobID] = forceStatus
watcher.Add(watch.Item{Alloc: alloc.ID})
watcher.Add(watch.Item{AllocEval: alloc.EvalID})
watcher.Add(watch.Item{AllocJob: alloc.JobID})
watcher.Add(watch.Item{AllocNode: alloc.NodeID})
}
// Update the indexes
if err := txn.Insert("index", &IndexEntry{"allocs", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
}
2016-01-09 02:22:59 +00:00
// Set the job's status
2016-01-12 01:34:25 +00:00
if err := s.setJobStatuses(index, watcher, txn, jobs, false); err != nil {
2016-01-09 02:22:59 +00:00
return fmt.Errorf("setting job status failed: %v", err)
}
txn.Defer(func() { s.watch.notify(watcher) })
txn.Commit()
return nil
}
2015-09-07 03:51:01 +00:00
// AllocByID is used to lookup an allocation by its ID
func (s *StateStore) AllocByID(id string) (*structs.Allocation, error) {
txn := s.db.Txn(false)
existing, err := txn.First("allocs", "id", id)
if err != nil {
return nil, fmt.Errorf("alloc lookup failed: %v", err)
}
if existing != nil {
return existing.(*structs.Allocation), nil
}
return nil, nil
}
// AllocsByIDPrefix is used to lookup allocs by prefix
func (s *StateStore) AllocsByIDPrefix(id string) (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
iter, err := txn.Get("allocs", "id_prefix", id)
if err != nil {
return nil, fmt.Errorf("alloc lookup failed: %v", err)
}
return iter, nil
}
// AllocsByNode returns all the allocations by node
func (s *StateStore) AllocsByNode(node string) ([]*structs.Allocation, error) {
txn := s.db.Txn(false)
// Get an iterator over the node allocations, using only the
// node prefix which ignores the terminal status
iter, err := txn.Get("allocs", "node_prefix", node)
if err != nil {
return nil, err
}
var out []*structs.Allocation
for {
raw := iter.Next()
if raw == nil {
break
}
out = append(out, raw.(*structs.Allocation))
}
return out, nil
}
// AllocsByNode returns all the allocations by node and terminal status
func (s *StateStore) AllocsByNodeTerminal(node string, terminal bool) ([]*structs.Allocation, error) {
txn := s.db.Txn(false)
// Get an iterator over the node allocations
iter, err := txn.Get("allocs", "node", node, terminal)
if err != nil {
return nil, err
}
2015-08-07 00:36:10 +00:00
var out []*structs.Allocation
for {
raw := iter.Next()
if raw == nil {
break
}
out = append(out, raw.(*structs.Allocation))
}
return out, nil
}
// AllocsByJob returns all the allocations by job id
func (s *StateStore) AllocsByJob(jobID string) ([]*structs.Allocation, error) {
txn := s.db.Txn(false)
// Get an iterator over the node allocations
iter, err := txn.Get("allocs", "job", jobID)
if err != nil {
return nil, err
}
var out []*structs.Allocation
for {
raw := iter.Next()
if raw == nil {
break
}
out = append(out, raw.(*structs.Allocation))
}
return out, nil
}
// AllocsByEval returns all the allocations by eval id
func (s *StateStore) AllocsByEval(evalID string) ([]*structs.Allocation, error) {
txn := s.db.Txn(false)
// Get an iterator over the eval allocations
iter, err := txn.Get("allocs", "eval", evalID)
if err != nil {
return nil, err
}
var out []*structs.Allocation
for {
raw := iter.Next()
if raw == nil {
break
}
out = append(out, raw.(*structs.Allocation))
}
return out, nil
}
// Allocs returns an iterator over all the evaluations
func (s *StateStore) Allocs() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Walk the entire table
iter, err := txn.Get("allocs", "id")
if err != nil {
return nil, err
}
return iter, nil
}
// LastIndex returns the greatest index value for all indexes
func (s *StateStore) LatestIndex() (uint64, error) {
indexes, err := s.Indexes()
if err != nil {
return 0, err
}
var max uint64 = 0
for {
raw := indexes.Next()
if raw == nil {
break
}
// Prepare the request struct
idx := raw.(*IndexEntry)
// Determine the max
if idx.Value > max {
max = idx.Value
}
}
return max, nil
}
2015-09-07 03:51:01 +00:00
// Index finds the matching index value
func (s *StateStore) Index(name string) (uint64, error) {
txn := s.db.Txn(false)
// Lookup the first matching index
out, err := txn.First("index", "id", name)
if err != nil {
return 0, err
}
if out == nil {
return 0, nil
}
return out.(*IndexEntry).Value, nil
}
// Indexes returns an iterator over all the indexes
func (s *StateStore) Indexes() (memdb.ResultIterator, error) {
txn := s.db.Txn(false)
// Walk the entire nodes table
iter, err := txn.Get("index", "id")
if err != nil {
return nil, err
}
return iter, nil
}
2016-01-09 02:22:59 +00:00
// setJobStatuses is a helper for calling setJobStatus on multiple jobs by ID.
// It takes a map of job IDs to an optional forceStatus string. It returns an
// error if the job doesn't exist or setJobStatus fails.
2016-01-12 01:34:25 +00:00
func (s *StateStore) setJobStatuses(index uint64, watcher watch.Items, txn *memdb.Txn,
2016-01-09 02:22:59 +00:00
jobs map[string]string, evalDelete bool) error {
for job, forceStatus := range jobs {
existing, err := txn.First("jobs", "id", job)
if err != nil {
return fmt.Errorf("job lookup failed: %v", err)
}
if existing == nil {
continue
}
2016-01-12 01:34:25 +00:00
if err := s.setJobStatus(index, watcher, txn, existing.(*structs.Job), evalDelete, forceStatus); err != nil {
2016-01-09 02:22:59 +00:00
return err
}
}
return nil
}
// setJobStatus sets the status of the job by looking up associated evaluations
// and allocations. evalDelete should be set to true if setJobStatus is being
// called because an evaluation is being deleted (potentially because of garbage
// collection). If forceStatus is non-empty, the job's status will be set to the
// passed status.
2016-01-12 01:34:25 +00:00
func (s *StateStore) setJobStatus(index uint64, watcher watch.Items, txn *memdb.Txn,
2016-01-09 02:22:59 +00:00
job *structs.Job, evalDelete bool, forceStatus string) error {
2016-01-12 01:34:25 +00:00
// Capture the current status so we can check if there is a change
oldStatus := job.Status
newStatus := forceStatus
// If forceStatus is not set, compute the jobs status.
if forceStatus == "" {
var err error
newStatus, err = s.getJobStatus(txn, job, evalDelete)
if err != nil {
return err
}
}
// Fast-path if nothing has changed.
if oldStatus == newStatus {
return nil
}
// The job has changed, so add to watcher.
2016-01-09 02:22:59 +00:00
watcher.Add(watch.Item{Table: "jobs"})
watcher.Add(watch.Item{Job: job.ID})
2016-01-12 01:34:25 +00:00
// Copy and update the existing job
updated := job.Copy()
updated.Status = newStatus
updated.ModifyIndex = index
// Insert the job
if err := txn.Insert("jobs", updated); err != nil {
return fmt.Errorf("job insert failed: %v", err)
}
if err := txn.Insert("index", &IndexEntry{"jobs", index}); err != nil {
return fmt.Errorf("index update failed: %v", err)
2016-01-09 02:22:59 +00:00
}
2016-01-12 01:34:25 +00:00
return nil
}
2016-01-09 02:22:59 +00:00
2016-01-12 01:34:25 +00:00
func (s *StateStore) getJobStatus(txn *memdb.Txn, job *structs.Job, evalDelete bool) (string, error) {
2016-01-09 02:22:59 +00:00
allocs, err := txn.Get("allocs", "job", job.ID)
if err != nil {
2016-01-12 01:34:25 +00:00
return "", err
2016-01-09 02:22:59 +00:00
}
// If there is a non-terminal allocation, the job is running.
hasAlloc := false
for alloc := allocs.Next(); alloc != nil; alloc = allocs.Next() {
hasAlloc = true
if !alloc.(*structs.Allocation).TerminalStatus() {
2016-01-12 01:34:25 +00:00
return structs.JobStatusRunning, nil
2016-01-09 02:22:59 +00:00
}
}
evals, err := txn.Get("evals", "job", job.ID)
if err != nil {
2016-01-12 01:34:25 +00:00
return "", err
2016-01-09 02:22:59 +00:00
}
hasEval := false
for eval := evals.Next(); eval != nil; eval = evals.Next() {
hasEval = true
if !eval.(*structs.Evaluation).TerminalStatus() {
2016-01-12 01:34:25 +00:00
return structs.JobStatusPending, nil
2016-01-09 02:22:59 +00:00
}
}
// The job is dead if all the allocations and evals are terminal or if there
// are no evals because of garbage collection.
if evalDelete || hasEval || hasAlloc {
2016-01-12 01:34:25 +00:00
return structs.JobStatusDead, nil
2016-01-09 02:22:59 +00:00
}
// If there are no allocations or evaluations it is a new job. If the job is
// periodic, we mark it as running as it will never have an
// allocation/evaluation against it.
if job.IsPeriodic() {
2016-01-12 01:34:25 +00:00
return structs.JobStatusRunning, nil
2016-01-09 02:22:59 +00:00
}
2016-01-12 01:34:25 +00:00
return structs.JobStatusPending, nil
2016-01-09 02:22:59 +00:00
}
// updateSummaryWithJob creates or updates job summaries when new jobs are
// upserted or existing ones are updated
func (s *StateStore) updateSummaryWithJob(job *structs.Job, txn *memdb.Txn) error {
existing, err := s.JobSummaryByID(job.ID)
if err != nil {
return fmt.Errorf("unable to retrieve summary for job: %v", err)
}
if existing == nil {
existing = &structs.JobSummary{
JobID: job.ID,
Summary: make(map[string]structs.TaskGroupSummary),
}
}
for _, tg := range job.TaskGroups {
if summary, ok := existing.Summary[tg.Name]; !ok {
newSummary := structs.TaskGroupSummary{
Queued: tg.Count,
Complete: 0,
Failed: 0,
Running: 0,
Starting: 0,
}
existing.Summary[tg.Name] = newSummary
} else if summary.Queued > tg.Count {
summary.Queued = tg.Count
existing.Summary[tg.Name] = summary
}
}
if err := txn.Insert("jobsummary", existing); err != nil {
return err
}
return nil
}
// updateSummaryWithAlloc updates the job summary when allocations are updated
// or inserted
func (s *StateStore) updateSummaryWithAlloc(newAlloc *structs.Allocation,
existingAlloc *structs.Allocation, txn *memdb.Txn) error {
existing, err := s.JobSummaryByID(newAlloc.JobID)
if err != nil {
return fmt.Errorf("lookup of job summary failed: %v", err)
}
// If we can't find an existing job summary entry then we are not going to create a
// new job summary entry for an allocation with that job id since we don't
// know the task group counts for that job
// TODO May be we can query the job and scan all the allocations for that
// job and create the summary before applying the change of summary state
// that this allocation would cause.
if existing == nil {
return nil
}
tgSummary, ok := existing.Summary[newAlloc.TaskGroup]
if !ok {
return nil
}
if existingAlloc == nil {
switch newAlloc.DesiredStatus {
case structs.AllocDesiredStatusFailed:
tgSummary.Failed += 1
case structs.AllocDesiredStatusStop:
tgSummary.Complete += 1
}
switch newAlloc.ClientStatus {
case structs.AllocClientStatusPending:
tgSummary.Starting += 1
case structs.AllocClientStatusRunning:
tgSummary.Running += 1
}
tgSummary.Queued -= 1
} else if existingAlloc.ClientStatus != newAlloc.ClientStatus {
// Incrementing the clint of the bin of the current state
switch newAlloc.ClientStatus {
case structs.AllocClientStatusRunning:
tgSummary.Running += 1
case structs.AllocClientStatusFailed:
tgSummary.Failed += 1
case structs.AllocClientStatusPending:
tgSummary.Starting += 1
case structs.AllocClientStatusComplete:
tgSummary.Complete += 1
}
// Decrementing the count of the bin of the last state
switch existingAlloc.ClientStatus {
case structs.AllocClientStatusRunning:
tgSummary.Running -= 1
case structs.AllocClientStatusFailed:
tgSummary.Failed -= 1
case structs.AllocClientStatusPending:
tgSummary.Starting -= 1
case structs.AllocClientStatusComplete:
tgSummary.Complete -= 1
}
} else if existingAlloc.DesiredStatus != newAlloc.DesiredStatus {
shouldDecrementStarting := false
switch newAlloc.DesiredStatus {
case structs.AllocDesiredStatusFailed:
tgSummary.Failed += 1
shouldDecrementStarting = true
case structs.AllocDesiredStatusStop:
tgSummary.Complete += 1
shouldDecrementStarting = true
}
if shouldDecrementStarting {
if newAlloc.ClientStatus == structs.AllocClientStatusPending {
tgSummary.Starting -= 1
}
}
}
existing.Summary[newAlloc.TaskGroup] = tgSummary
if err := txn.Insert("jobsummary", existing); err != nil {
return fmt.Errorf("inserting job summary failed: %v", err)
}
return nil
}
// StateSnapshot is used to provide a point-in-time snapshot
type StateSnapshot struct {
StateStore
}
// StateRestore is used to optimize the performance when
// restoring state by only using a single large transaction
// instead of thousands of sub transactions
type StateRestore struct {
txn *memdb.Txn
watch *stateWatch
items watch.Items
}
// Abort is used to abort the restore operation
func (s *StateRestore) Abort() {
s.txn.Abort()
}
// Commit is used to commit the restore operation
func (s *StateRestore) Commit() {
s.txn.Defer(func() { s.watch.notify(s.items) })
s.txn.Commit()
}
// NodeRestore is used to restore a node
func (r *StateRestore) NodeRestore(node *structs.Node) error {
r.items.Add(watch.Item{Table: "nodes"})
2015-10-30 04:42:41 +00:00
r.items.Add(watch.Item{Node: node.ID})
if err := r.txn.Insert("nodes", node); err != nil {
return fmt.Errorf("node insert failed: %v", err)
}
return nil
}
2015-07-07 16:41:05 +00:00
// JobRestore is used to restore a job
func (r *StateRestore) JobRestore(job *structs.Job) error {
r.items.Add(watch.Item{Table: "jobs"})
2015-10-30 04:42:41 +00:00
r.items.Add(watch.Item{Job: job.ID})
2015-07-07 16:41:05 +00:00
if err := r.txn.Insert("jobs", job); err != nil {
return fmt.Errorf("job insert failed: %v", err)
2015-07-07 16:41:05 +00:00
}
return nil
}
2015-07-23 22:43:06 +00:00
// EvalRestore is used to restore an evaluation
func (r *StateRestore) EvalRestore(eval *structs.Evaluation) error {
r.items.Add(watch.Item{Table: "evals"})
2015-10-30 04:42:41 +00:00
r.items.Add(watch.Item{Eval: eval.ID})
2015-07-23 22:43:06 +00:00
if err := r.txn.Insert("evals", eval); err != nil {
return fmt.Errorf("eval insert failed: %v", err)
}
return nil
}
// AllocRestore is used to restore an allocation
func (r *StateRestore) AllocRestore(alloc *structs.Allocation) error {
r.items.Add(watch.Item{Table: "allocs"})
2015-10-30 15:27:47 +00:00
r.items.Add(watch.Item{Alloc: alloc.ID})
r.items.Add(watch.Item{AllocEval: alloc.EvalID})
r.items.Add(watch.Item{AllocJob: alloc.JobID})
r.items.Add(watch.Item{AllocNode: alloc.NodeID})
if err := r.txn.Insert("allocs", alloc); err != nil {
return fmt.Errorf("alloc insert failed: %v", err)
}
return nil
}
2015-07-23 22:43:06 +00:00
// IndexRestore is used to restore an index
func (r *StateRestore) IndexRestore(idx *IndexEntry) error {
if err := r.txn.Insert("index", idx); err != nil {
return fmt.Errorf("index insert failed: %v", err)
}
return nil
}
2015-12-07 23:58:17 +00:00
// PeriodicLaunchRestore is used to restore a periodic launch.
func (r *StateRestore) PeriodicLaunchRestore(launch *structs.PeriodicLaunch) error {
r.items.Add(watch.Item{Table: "periodic_launch"})
r.items.Add(watch.Item{Job: launch.ID})
if err := r.txn.Insert("periodic_launch", launch); err != nil {
return fmt.Errorf("periodic launch insert failed: %v", err)
}
return nil
}
// JobSummaryRestore is used to restore a job summary
func (r *StateRestore) JobSummaryRestore(jobSummary *structs.JobSummary) error {
if err := r.txn.Insert("jobsummary", jobSummary); err != nil {
return fmt.Errorf("job summary insert failed: %v", err)
}
return nil
}
// stateWatch holds shared state for watching updates. This is
// outside of StateStore so it can be shared with snapshots.
type stateWatch struct {
items map[watch.Item]*NotifyGroup
l sync.Mutex
}
// newStateWatch creates a new stateWatch for change notification.
func newStateWatch() *stateWatch {
return &stateWatch{
items: make(map[watch.Item]*NotifyGroup),
}
}
// watch subscribes a channel to the given watch items.
func (w *stateWatch) watch(items watch.Items, ch chan struct{}) {
w.l.Lock()
defer w.l.Unlock()
for item, _ := range items {
grp, ok := w.items[item]
if !ok {
grp = new(NotifyGroup)
w.items[item] = grp
}
grp.Wait(ch)
}
}
// stopWatch unsubscribes a channel from the given watch items.
func (w *stateWatch) stopWatch(items watch.Items, ch chan struct{}) {
w.l.Lock()
defer w.l.Unlock()
for item, _ := range items {
if grp, ok := w.items[item]; ok {
grp.Clear(ch)
if grp.Empty() {
delete(w.items, item)
}
}
}
}
// notify is used to fire notifications on the given watch items.
func (w *stateWatch) notify(items watch.Items) {
w.l.Lock()
defer w.l.Unlock()
for wi, _ := range items {
if grp, ok := w.items[wi]; ok {
grp.Notify()
}
}
}