influxdb/query/execute/executor.go

281 lines
6.6 KiB
Go
Raw Normal View History

// Package execute contains the implementation of the execution phase in the query engine.
2018-05-21 21:13:54 +00:00
package execute
import (
"context"
"fmt"
"runtime/debug"
"time"
2018-05-21 21:13:54 +00:00
"github.com/influxdata/platform"
2018-05-21 21:20:06 +00:00
"github.com/influxdata/platform/query"
"github.com/influxdata/platform/query/plan"
2018-05-21 21:13:54 +00:00
"github.com/pkg/errors"
"go.uber.org/zap"
"go.uber.org/zap/zapcore"
2018-05-21 21:13:54 +00:00
)
type Executor interface {
Execute(ctx context.Context, orgID platform.ID, p *plan.PlanSpec, a *Allocator) (map[string]query.Result, error)
2018-05-21 21:13:54 +00:00
}
type executor struct {
deps Dependencies
logger *zap.Logger
2018-05-21 21:13:54 +00:00
}
func NewExecutor(deps Dependencies, logger *zap.Logger) Executor {
if logger == nil {
logger = zap.NewNop()
}
2018-05-21 21:13:54 +00:00
e := &executor{
deps: deps,
logger: logger,
2018-05-21 21:13:54 +00:00
}
return e
}
type streamContext struct {
bounds Bounds
}
func newStreamContext(b Bounds) streamContext {
return streamContext{
bounds: b,
}
}
func (ctx streamContext) Bounds() Bounds {
return ctx.bounds
}
2018-05-21 21:13:54 +00:00
type executionState struct {
p *plan.PlanSpec
deps Dependencies
orgID platform.ID
2018-05-21 21:13:54 +00:00
alloc *Allocator
resources query.ResourceManagement
results map[string]query.Result
2018-05-21 21:13:54 +00:00
sources []Source
transports []Transport
dispatcher *poolDispatcher
logger *zap.Logger
2018-05-21 21:13:54 +00:00
}
func (e *executor) Execute(ctx context.Context, orgID platform.ID, p *plan.PlanSpec, a *Allocator) (map[string]query.Result, error) {
es, err := e.createExecutionState(ctx, orgID, p, a)
2018-05-21 21:13:54 +00:00
if err != nil {
return nil, errors.Wrap(err, "failed to initialize execute state")
}
es.logger = e.logger
2018-05-21 21:13:54 +00:00
es.do(ctx)
return es.results, nil
}
func validatePlan(p *plan.PlanSpec) error {
if p.Resources.ConcurrencyQuota == 0 {
return errors.New("plan must have a non-zero concurrency quota")
}
return nil
}
func (e *executor) createExecutionState(ctx context.Context, orgID platform.ID, p *plan.PlanSpec, a *Allocator) (*executionState, error) {
2018-05-21 21:13:54 +00:00
if err := validatePlan(p); err != nil {
return nil, errors.Wrap(err, "invalid plan")
}
// Set allocation limit
a.Limit = p.Resources.MemoryBytesQuota
2018-05-21 21:13:54 +00:00
es := &executionState{
orgID: orgID,
p: p,
deps: e.deps,
alloc: a,
2018-05-21 21:13:54 +00:00
resources: p.Resources,
results: make(map[string]query.Result, len(p.Results)),
2018-05-21 21:13:54 +00:00
// TODO(nathanielc): Have the planner specify the dispatcher throughput
dispatcher: newPoolDispatcher(10, e.logger),
2018-05-21 21:13:54 +00:00
}
nodes := make(map[plan.ProcedureID]Node, len(p.Procedures))
for name, yield := range p.Results {
ds, err := es.createNode(ctx, p.Procedures[yield.ID], nodes)
if err != nil {
return nil, err
}
r := newResult(name, yield)
2018-05-21 21:13:54 +00:00
ds.AddTransformation(r)
es.results[name] = r
}
return es, nil
}
// DefaultTriggerSpec defines the triggering that should be used for datasets
// whose parent transformation is not a windowing transformation.
var DefaultTriggerSpec = query.AfterWatermarkTriggerSpec{}
type triggeringSpec interface {
TriggerSpec() query.TriggerSpec
}
func (es *executionState) createNode(ctx context.Context, pr *plan.Procedure, nodes map[plan.ProcedureID]Node) (Node, error) {
// Check if we already created this node
if n, ok := nodes[pr.ID]; ok {
return n, nil
}
2018-05-21 21:13:54 +00:00
// Build execution context
ec := executionContext{
es: es,
streamContext: newStreamContext(Bounds{
Start: resolveTime(pr.Bounds.Start, es.p.Now),
Stop: resolveTime(pr.Bounds.Stop, es.p.Now),
}),
2018-05-21 21:13:54 +00:00
}
2018-05-21 21:13:54 +00:00
if len(pr.Parents) > 0 {
ec.parents = make([]DatasetID, len(pr.Parents))
for i, parentID := range pr.Parents {
ec.parents[i] = DatasetID(parentID)
}
}
// If source create source
if createS, ok := procedureToSource[pr.Spec.Kind()]; ok {
s, err := createS(pr.Spec, DatasetID(pr.ID), ec)
if err != nil {
return nil, err
}
es.sources = append(es.sources, s)
nodes[pr.ID] = s
return s, nil
}
createT, ok := procedureToTransformation[pr.Spec.Kind()]
if !ok {
return nil, fmt.Errorf("unsupported procedure %v", pr.Spec.Kind())
}
// Create the transformation
t, ds, err := createT(DatasetID(pr.ID), AccumulatingMode, pr.Spec, ec)
if err != nil {
return nil, err
}
nodes[pr.ID] = ds
// Setup triggering
var ts query.TriggerSpec = DefaultTriggerSpec
if t, ok := pr.Spec.(triggeringSpec); ok {
ts = t.TriggerSpec()
}
ds.SetTriggerSpec(ts)
// Recurse creating parents
for _, parentID := range pr.Parents {
parent, err := es.createNode(ctx, es.p.Procedures[parentID], nodes)
if err != nil {
return nil, err
}
transport := newConescutiveTransport(es.dispatcher, t)
es.transports = append(es.transports, transport)
parent.AddTransformation(transport)
}
return ds, nil
}
func (es *executionState) abort(err error) {
for _, r := range es.results {
r.(*result).abort(err)
}
}
func (es *executionState) do(ctx context.Context) {
for _, src := range es.sources {
go func(src Source) {
// Setup panic handling on the source goroutines
defer func() {
if e := recover(); e != nil {
// We had a panic, abort the entire execution.
var err error
switch e := e.(type) {
case error:
err = e
default:
err = fmt.Errorf("%v", e)
}
es.abort(fmt.Errorf("panic: %v\n%s", err, debug.Stack()))
if entry := es.logger.Check(zapcore.InfoLevel, "Execute source panic"); entry != nil {
entry.Stack = string(debug.Stack())
entry.Write(zap.Error(err))
}
2018-05-21 21:13:54 +00:00
}
}()
src.Run(ctx)
}(src)
}
es.dispatcher.Start(es.resources.ConcurrencyQuota, ctx)
go func() {
// Wait for all transports to finish
for _, t := range es.transports {
select {
case <-t.Finished():
case <-ctx.Done():
es.abort(errors.New("context done"))
case err := <-es.dispatcher.Err():
if err != nil {
es.abort(err)
}
}
}
// Check for any errors on the dispatcher
err := es.dispatcher.Stop()
if err != nil {
es.abort(err)
}
}()
}
// Need a unique stream context per execution context
2018-05-21 21:13:54 +00:00
type executionContext struct {
es *executionState
parents []DatasetID
streamContext streamContext
2018-05-21 21:13:54 +00:00
}
// Satisfy the ExecutionContext interface
func (ec executionContext) OrganizationID() platform.ID {
2018-05-21 21:13:54 +00:00
return ec.es.orgID
}
func resolveTime(qt query.Time, now time.Time) Time {
return Time(qt.Time(now).UnixNano())
}
2018-05-21 21:13:54 +00:00
func (ec executionContext) ResolveTime(qt query.Time) Time {
return resolveTime(qt, ec.es.p.Now)
2018-05-21 21:13:54 +00:00
}
func (ec executionContext) StreamContext() StreamContext {
return ec.streamContext
2018-05-21 21:13:54 +00:00
}
func (ec executionContext) Allocator() *Allocator {
return ec.es.alloc
}
func (ec executionContext) Parents() []DatasetID {
return ec.parents
}
func (ec executionContext) ConvertID(id plan.ProcedureID) DatasetID {
return DatasetID(id)
}
func (ec executionContext) Dependencies() Dependencies {
return ec.es.deps
}