prepare the transition to #Plan.context

This change helps the transition between `dagger input` and `#Plan.context`.

In summary, the codebase now relies on a *context* for execution with mapping to *IDs*.
In the future, *context* will come from a `#Plan.context`.
In the meantime, a bridge converts `dagger input` to a plan context. This allows both *old* and *new* style configurations to co-exist with the same underlying engine.

- Implement `plancontext`. Context holds the execution context for a plan. Currently this includes the platform, local directories, secrets and services (e.g. unix/npipe).
- Contextual data can be registered at any point. In the future, this will be done by `#Plan.context`
- Migrated the `dagger input` codebase to register inputs in a `plancontext`
- Migrated low-level types/operations to the *Context ID* pattern.
  - `dagger.#Stream` now only includes an `id` (instead of `unix` path)
  - `dagger.#Secret` still includes only an ID, but now it's based off `plancontext`
  - `op.#Local` now only includes an `id` (instead of `path`, `include`, `exclude`.

Signed-off-by: Andrea Luzzardi <aluzzardi@gmail.com>
This commit is contained in:
Andrea Luzzardi 2021-11-16 16:13:45 -08:00
parent 88312f7f82
commit a61e8dcb62
27 changed files with 401 additions and 423 deletions

View File

@ -8,7 +8,6 @@ import (
"sync" "sync"
"github.com/containerd/containerd/platforms" "github.com/containerd/containerd/platforms"
"go.opentelemetry.io/otel"
"golang.org/x/sync/errgroup" "golang.org/x/sync/errgroup"
"github.com/rs/zerolog/log" "github.com/rs/zerolog/log"
@ -23,13 +22,12 @@ import (
"github.com/moby/buildkit/session" "github.com/moby/buildkit/session"
// docker output // docker output
"go.dagger.io/dagger/plancontext"
"go.dagger.io/dagger/util/buildkitd" "go.dagger.io/dagger/util/buildkitd"
"go.dagger.io/dagger/util/progressui" "go.dagger.io/dagger/util/progressui"
"go.dagger.io/dagger/compiler" "go.dagger.io/dagger/compiler"
"go.dagger.io/dagger/environment"
"go.dagger.io/dagger/solver" "go.dagger.io/dagger/solver"
"go.dagger.io/dagger/state"
) )
// Client is a dagger client // Client is a dagger client
@ -74,36 +72,31 @@ func New(ctx context.Context, host string, cfg Config) (*Client, error) {
}, nil }, nil
} }
type DoFunc func(context.Context, *environment.Environment, solver.Solver) error type DoFunc func(context.Context, solver.Solver) error
// FIXME: return completed *Route, instead of *compiler.Value // FIXME: return completed *Route, instead of *compiler.Value
func (c *Client) Do(ctx context.Context, state *state.State, fn DoFunc) error { func (c *Client) Do(ctx context.Context, pctx *plancontext.Context, fn DoFunc) error {
lg := log.Ctx(ctx) lg := log.Ctx(ctx)
eg, gctx := errgroup.WithContext(ctx) eg, gctx := errgroup.WithContext(ctx)
env, err := environment.New(state)
if err != nil {
return err
}
// Spawn print function // Spawn print function
events := make(chan *bk.SolveStatus) events := make(chan *bk.SolveStatus)
eg.Go(func() error { eg.Go(func() error {
// Create a background context so that logging will not be cancelled // Create a background context so that logging will not be cancelled
// with the main context. // with the main context.
dispCtx := lg.WithContext(context.Background()) dispCtx := lg.WithContext(context.Background())
return c.logSolveStatus(dispCtx, state, events) return c.logSolveStatus(dispCtx, pctx, events)
}) })
// Spawn build function // Spawn build function
eg.Go(func() error { eg.Go(func() error {
return c.buildfn(gctx, state, env, fn, events) return c.buildfn(gctx, pctx, fn, events)
}) })
return eg.Wait() return eg.Wait()
} }
func (c *Client) buildfn(ctx context.Context, st *state.State, env *environment.Environment, fn DoFunc, ch chan *bk.SolveStatus) error { func (c *Client) buildfn(ctx context.Context, pctx *plancontext.Context, fn DoFunc, ch chan *bk.SolveStatus) error {
wg := sync.WaitGroup{} wg := sync.WaitGroup{}
// Close output channel // Close output channel
@ -115,25 +108,21 @@ func (c *Client) buildfn(ctx context.Context, st *state.State, env *environment.
lg := log.Ctx(ctx) lg := log.Ctx(ctx)
// Scan local dirs to grant access
localdirs, err := env.LocalDirs()
if err != nil {
return err
}
// buildkit auth provider (registry) // buildkit auth provider (registry)
auth := solver.NewRegistryAuthProvider() auth := solver.NewRegistryAuthProvider()
// session (secrets & store) localdirs := map[string]string{}
secretsStore := solver.NewSecretsStoreProvider(st) for _, dir := range pctx.Directories.List() {
localdirs[dir.Path] = dir.Path
}
// Setup solve options // Setup solve options
opts := bk.SolveOpt{ opts := bk.SolveOpt{
LocalDirs: localdirs, LocalDirs: localdirs,
Session: []session.Attachable{ Session: []session.Attachable{
auth, auth,
secretsStore.Secrets, solver.NewSecretsStoreProvider(pctx),
solver.NewDockerSocketProvider(), solver.NewDockerSocketProvider(pctx),
}, },
CacheExports: c.cfg.CacheExports, CacheExports: c.cfg.CacheExports,
CacheImports: c.cfg.CacheImports, CacheImports: c.cfg.CacheImports,
@ -167,12 +156,11 @@ func (c *Client) buildfn(ctx context.Context, st *state.State, env *environment.
resp, err := c.c.Build(ctx, opts, "", func(ctx context.Context, gw bkgw.Client) (*bkgw.Result, error) { resp, err := c.c.Build(ctx, opts, "", func(ctx context.Context, gw bkgw.Client) (*bkgw.Result, error) {
s := solver.New(solver.Opts{ s := solver.New(solver.Opts{
Control: c.c, Control: c.c,
Gateway: gw, Gateway: gw,
Events: eventsCh, Events: eventsCh,
Auth: auth, Auth: auth,
SecretsStore: secretsStore, NoCache: c.cfg.NoCache,
NoCache: c.cfg.NoCache,
}) })
// Close events channel // Close events channel
@ -180,28 +168,12 @@ func (c *Client) buildfn(ctx context.Context, st *state.State, env *environment.
// Compute output overlay // Compute output overlay
if fn != nil { if fn != nil {
if err := fn(ctx, env, s); err != nil { if err := fn(ctx, s); err != nil {
return nil, compiler.Err(err) return nil, compiler.Err(err)
} }
} }
// Export environment to a cue directory ref, err := s.Solve(ctx, llb.Scratch(), platforms.DefaultSpec())
// FIXME: this should be elsewhere
lg.Debug().Msg("exporting environment")
tr := otel.Tracer("client")
_, span := tr.Start(ctx, "environment.Export")
defer span.End()
computed := env.Computed().JSON().PrettyString()
st := llb.
Scratch().
File(
llb.Mkfile("computed.json", 0600, []byte(computed)),
llb.WithCustomName("[internal] serializing computed values"),
)
ref, err := s.Solve(ctx, st, platforms.DefaultSpec())
if err != nil { if err != nil {
return nil, err return nil, err
} }
@ -224,7 +196,7 @@ func (c *Client) buildfn(ctx context.Context, st *state.State, env *environment.
return nil return nil
} }
func (c *Client) logSolveStatus(ctx context.Context, st *state.State, ch chan *bk.SolveStatus) error { func (c *Client) logSolveStatus(ctx context.Context, pctx *plancontext.Context, ch chan *bk.SolveStatus) error {
parseName := func(v *bk.Vertex) (string, string) { parseName := func(v *bk.Vertex) (string, string) {
// Pattern: `@name@ message`. Minimal length is len("@X@ ") // Pattern: `@name@ message`. Minimal length is len("@X@ ")
if len(v.Name) < 2 || !strings.HasPrefix(v.Name, "@") { if len(v.Name) < 2 || !strings.HasPrefix(v.Name, "@") {
@ -241,13 +213,11 @@ func (c *Client) logSolveStatus(ctx context.Context, st *state.State, ch chan *b
} }
// Just like sprintf, but redacts secrets automatically // Just like sprintf, but redacts secrets automatically
secrets := pctx.Secrets.List()
secureSprintf := func(format string, a ...interface{}) string { secureSprintf := func(format string, a ...interface{}) string {
s := fmt.Sprintf(format, a...) s := fmt.Sprintf(format, a...)
for _, i := range st.Inputs { for _, secret := range secrets {
if i.Secret == nil { s = strings.ReplaceAll(s, secret.PlainText, "***")
continue
}
s = strings.ReplaceAll(s, i.Secret.PlainText(), "***")
} }
return s return s
} }

View File

@ -15,6 +15,7 @@ import (
"go.dagger.io/dagger/cmd/dagger/logger" "go.dagger.io/dagger/cmd/dagger/logger"
"go.dagger.io/dagger/compiler" "go.dagger.io/dagger/compiler"
"go.dagger.io/dagger/environment" "go.dagger.io/dagger/environment"
"go.dagger.io/dagger/plancontext"
"go.dagger.io/dagger/solver" "go.dagger.io/dagger/solver"
"go.dagger.io/dagger/state" "go.dagger.io/dagger/state"
"go.mozilla.org/sops/v3" "go.mozilla.org/sops/v3"
@ -43,6 +44,7 @@ var computeCmd = &cobra.Command{
doneCh := common.TrackCommand(ctx, cmd) doneCh := common.TrackCommand(ctx, cmd)
st := &state.State{ st := &state.State{
Context: plancontext.New(),
Name: "FIXME", Name: "FIXME",
Platform: platforms.Format(specs.Platform{OS: "linux", Architecture: "amd64"}), Platform: platforms.Format(specs.Platform{OS: "linux", Architecture: "amd64"}),
Path: args[0], Path: args[0],
@ -191,7 +193,12 @@ var computeCmd = &cobra.Command{
lg.Fatal().Err(err).Msg("failed to compile inputs") lg.Fatal().Err(err).Msg("failed to compile inputs")
} }
err = cl.Do(ctx, st, func(ctx context.Context, env *environment.Environment, s solver.Solver) error { env, err := environment.New(st)
if err != nil {
lg.Fatal().Msg("unable to create environment")
}
err = cl.Do(ctx, env.Context(), func(ctx context.Context, s solver.Solver) error {
// check that all inputs are set // check that all inputs are set
checkInputs(ctx, env) checkInputs(ctx, env)

View File

@ -77,8 +77,13 @@ var editCmd = &cobra.Command{
st.Plan = newState.Plan st.Plan = newState.Plan
st.Inputs = newState.Inputs st.Inputs = newState.Inputs
env, err := environment.New(st)
if err != nil {
lg.Fatal().Msg("unable to create environment")
}
cl := common.NewClient(ctx) cl := common.NewClient(ctx)
err = cl.Do(ctx, st, func(ctx context.Context, env *environment.Environment, s solver.Solver) error { err = cl.Do(ctx, env.Context(), func(ctx context.Context, s solver.Solver) error {
// check for cue errors by scanning all the inputs // check for cue errors by scanning all the inputs
_, err := env.ScanInputs(ctx, true) _, err := env.ScanInputs(ctx, true)
if err != nil { if err != nil {

View File

@ -1,33 +0,0 @@
package input
import (
"github.com/spf13/cobra"
"github.com/spf13/viper"
"go.dagger.io/dagger/cmd/dagger/logger"
"go.dagger.io/dagger/state"
)
var containerCmd = &cobra.Command{
Use: "container TARGET CONTAINER-IMAGE",
Short: "Add a container image as input artifact",
Args: cobra.ExactArgs(2),
PreRun: func(cmd *cobra.Command, args []string) {
// Fix Viper bug for duplicate flags:
// https://github.com/spf13/viper/issues/233
if err := viper.BindPFlags(cmd.Flags()); err != nil {
panic(err)
}
},
Run: func(cmd *cobra.Command, args []string) {
lg := logger.New()
ctx := lg.WithContext(cmd.Context())
updateEnvironmentInput(ctx, cmd, args[0], state.DockerInput(args[1]))
},
}
func init() {
if err := viper.BindPFlags(containerCmd.Flags()); err != nil {
panic(err)
}
}

View File

@ -41,8 +41,13 @@ var listCmd = &cobra.Command{
doneCh := common.TrackProjectCommand(ctx, cmd, project, st) doneCh := common.TrackProjectCommand(ctx, cmd, project, st)
c := common.NewClient(ctx) env, err := environment.New(st)
err := c.Do(ctx, st, func(ctx context.Context, env *environment.Environment, s solver.Solver) error { if err != nil {
lg.Fatal().Msg("unable to create environment")
}
cl := common.NewClient(ctx)
err = cl.Do(ctx, env.Context(), func(ctx context.Context, s solver.Solver) error {
inputs, err := env.ScanInputs(ctx, false) inputs, err := env.ScanInputs(ctx, false)
if err != nil { if err != nil {
return err return err

View File

@ -25,7 +25,6 @@ func init() {
Cmd.AddCommand( Cmd.AddCommand(
dirCmd, dirCmd,
gitCmd, gitCmd,
containerCmd,
secretCmd, secretCmd,
textCmd, textCmd,
jsonCmd, jsonCmd,
@ -52,11 +51,15 @@ func updateEnvironmentInput(ctx context.Context, cmd *cobra.Command, target stri
Value: target, Value: target,
}) })
cl := common.NewClient(ctx)
st.SetInput(target, input) st.SetInput(target, input)
err := cl.Do(ctx, st, func(ctx context.Context, env *environment.Environment, s solver.Solver) error { env, err := environment.New(st)
if err != nil {
lg.Fatal().Msg("unable to create environment")
}
cl := common.NewClient(ctx)
err = cl.Do(ctx, env.Context(), func(ctx context.Context, s solver.Solver) error {
// the inputs are set, check for cue errors by scanning all the inputs // the inputs are set, check for cue errors by scanning all the inputs
_, err := env.ScanInputs(ctx, true) _, err := env.ScanInputs(ctx, true)
if err != nil { if err != nil {

View File

@ -40,8 +40,13 @@ var listCmd = &cobra.Command{
doneCh := common.TrackProjectCommand(ctx, cmd, project, st) doneCh := common.TrackProjectCommand(ctx, cmd, project, st)
env, err := environment.New(st)
if err != nil {
lg.Fatal().Msg("unable to create environment")
}
cl := common.NewClient(ctx) cl := common.NewClient(ctx)
err := cl.Do(ctx, st, func(ctx context.Context, env *environment.Environment, s solver.Solver) error { err = cl.Do(ctx, env.Context(), func(ctx context.Context, s solver.Solver) error {
return ListOutputs(ctx, env, true) return ListOutputs(ctx, env, true)
}) })

View File

@ -61,7 +61,12 @@ var upCmd = &cobra.Command{
cl := common.NewClient(ctx) cl := common.NewClient(ctx)
err = cl.Do(ctx, st, func(ctx context.Context, env *environment.Environment, s solver.Solver) error { env, err := environment.New(st)
if err != nil {
lg.Fatal().Msg("unable to create environment")
}
err = cl.Do(ctx, env.Context(), func(ctx context.Context, s solver.Solver) error {
// check that all inputs are set // check that all inputs are set
if err := checkInputs(ctx, env); err != nil { if err := checkInputs(ctx, env); err != nil {
return err return err

View File

@ -3,13 +3,11 @@ package environment
import ( import (
"context" "context"
"fmt" "fmt"
"path/filepath"
"cuelang.org/go/cue" "cuelang.org/go/cue"
cueflow "cuelang.org/go/tools/flow" cueflow "cuelang.org/go/tools/flow"
"github.com/containerd/containerd/platforms"
specs "github.com/opencontainers/image-spec/specs-go/v1"
"go.dagger.io/dagger/compiler" "go.dagger.io/dagger/compiler"
"go.dagger.io/dagger/plancontext"
"go.dagger.io/dagger/solver" "go.dagger.io/dagger/solver"
"go.dagger.io/dagger/state" "go.dagger.io/dagger/state"
@ -74,59 +72,8 @@ func (e *Environment) Computed() *compiler.Value {
return e.computed return e.computed
} }
// Scan all scripts in the environment for references to local directories (do:"local"), func (e *Environment) Context() *plancontext.Context {
// and return all referenced directory names. return e.state.Context
// This is used by clients to grant access to local directories when they are referenced
// by user-specified scripts.
func (e *Environment) LocalDirs() (map[string]string, error) {
dirs := map[string]string{}
localdirs := func(code *compiler.Value) error {
return Analyze(
func(op *compiler.Value) error {
do, err := op.Lookup("do").String()
if err != nil {
return err
}
if do != "local" {
return nil
}
dir, err := op.Lookup("dir").String()
if err != nil {
return err
}
abs, err := filepath.Abs(dir)
if err != nil {
return err
}
dirs[dir] = abs
return nil
},
code,
)
}
// 1. Scan the environment state
// FIXME: use a common `flow` instance to avoid rescanning the tree.
src := compiler.NewValue()
if err := src.FillPath(cue.MakePath(), e.plan); err != nil {
return nil, err
}
if err := src.FillPath(cue.MakePath(), e.input); err != nil {
return nil, err
}
flow := cueflow.New(
&cueflow.Config{},
src.Cue(),
newTaskFunc(noOpRunner),
)
for _, t := range flow.Tasks() {
if err := localdirs(compiler.Wrap(t.Value())); err != nil {
return nil, err
}
}
return dirs, nil
} }
// Up missing values in environment configuration, and write them to state. // Up missing values in environment configuration, and write them to state.
@ -139,7 +86,7 @@ func (e *Environment) Up(ctx context.Context, s solver.Solver) error {
flow := cueflow.New( flow := cueflow.New(
&cueflow.Config{}, &cueflow.Config{},
e.src.Cue(), e.src.Cue(),
newTaskFunc(newPipelineRunner(e.computed, s, e.state.Platform)), NewTaskFunc(NewPipelineRunner(e.computed, s, e.state.Context)),
) )
if err := flow.Run(ctx); err != nil { if err := flow.Run(ctx); err != nil {
return err return err
@ -163,7 +110,7 @@ func (e *Environment) Down(ctx context.Context, _ *DownOpts) error {
type QueryOpts struct{} type QueryOpts struct{}
func newTaskFunc(runner cueflow.RunnerFunc) cueflow.TaskFunc { func NewTaskFunc(runner cueflow.RunnerFunc) cueflow.TaskFunc {
return func(flowVal cue.Value) (cueflow.Runner, error) { return func(flowVal cue.Value) (cueflow.Runner, error) {
v := compiler.Wrap(flowVal) v := compiler.Wrap(flowVal)
if !isComponent(v) { if !isComponent(v) {
@ -174,11 +121,7 @@ func newTaskFunc(runner cueflow.RunnerFunc) cueflow.TaskFunc {
} }
} }
func noOpRunner(t *cueflow.Task) error { func NewPipelineRunner(computed *compiler.Value, s solver.Solver, pctx *plancontext.Context) cueflow.RunnerFunc {
return nil
}
func newPipelineRunner(computed *compiler.Value, s solver.Solver, platform string) cueflow.RunnerFunc {
return cueflow.RunnerFunc(func(t *cueflow.Task) error { return cueflow.RunnerFunc(func(t *cueflow.Task) error {
ctx := t.Context() ctx := t.Context()
lg := log. lg := log.
@ -200,23 +143,7 @@ func newPipelineRunner(computed *compiler.Value, s solver.Solver, platform strin
} }
v := compiler.Wrap(t.Value()) v := compiler.Wrap(t.Value())
var pipelinePlatform specs.Platform p := NewPipeline(v, s, pctx)
if platform == "" {
pipelinePlatform = specs.Platform{OS: "linux", Architecture: "amd64"}
} else {
p, err := platforms.Parse(platform)
if err != nil {
// Record the error
span.AddEvent("command", trace.WithAttributes(
attribute.String("error", err.Error()),
))
return err
}
pipelinePlatform = p
}
p := NewPipeline(v, s, pipelinePlatform)
err := p.Run(ctx) err := p.Run(ctx)
if err != nil { if err != nil {
// Record the error // Record the error

View File

@ -23,11 +23,11 @@ import (
bkgw "github.com/moby/buildkit/frontend/gateway/client" bkgw "github.com/moby/buildkit/frontend/gateway/client"
bkpb "github.com/moby/buildkit/solver/pb" bkpb "github.com/moby/buildkit/solver/pb"
digest "github.com/opencontainers/go-digest" digest "github.com/opencontainers/go-digest"
specs "github.com/opencontainers/image-spec/specs-go/v1"
"github.com/rs/zerolog/log" "github.com/rs/zerolog/log"
"gopkg.in/yaml.v3" "gopkg.in/yaml.v3"
"go.dagger.io/dagger/compiler" "go.dagger.io/dagger/compiler"
"go.dagger.io/dagger/plancontext"
"go.dagger.io/dagger/solver" "go.dagger.io/dagger/solver"
) )
@ -46,17 +46,17 @@ type Pipeline struct {
name string name string
s solver.Solver s solver.Solver
state llb.State state llb.State
platform specs.Platform // Platform constraint pctx *plancontext.Context
result bkgw.Reference result bkgw.Reference
image dockerfile2llb.Image image dockerfile2llb.Image
computed *compiler.Value computed *compiler.Value
} }
func NewPipeline(code *compiler.Value, s solver.Solver, platform specs.Platform) *Pipeline { func NewPipeline(code *compiler.Value, s solver.Solver, pctx *plancontext.Context) *Pipeline {
return &Pipeline{ return &Pipeline{
code: code, code: code,
name: code.Path().String(), name: code.Path().String(),
platform: platform, pctx: pctx,
s: s, s: s,
state: llb.Scratch(), state: llb.Scratch(),
computed: compiler.NewValue(), computed: compiler.NewValue(),
@ -233,7 +233,7 @@ func (p *Pipeline) run(ctx context.Context) error {
// so that errors map to the correct cue path. // so that errors map to the correct cue path.
// FIXME: might as well change FS to make every operation // FIXME: might as well change FS to make every operation
// synchronous. // synchronous.
p.result, err = p.s.Solve(ctx, p.state, p.platform) p.result, err = p.s.Solve(ctx, p.state, p.pctx.Platform.Get())
if err != nil { if err != nil {
return err return err
} }
@ -339,7 +339,7 @@ func (p *Pipeline) Copy(ctx context.Context, op *compiler.Value, st llb.State) (
return st, err return st, err
} }
// Execute 'from' in a tmp pipeline, and use the resulting fs // Execute 'from' in a tmp pipeline, and use the resulting fs
from := NewPipeline(op.Lookup("from"), p.s, p.platform) from := NewPipeline(op.Lookup("from"), p.s, p.pctx)
if err := from.Run(ctx); err != nil { if err := from.Run(ctx); err != nil {
return st, err return st, err
} }
@ -361,50 +361,31 @@ func (p *Pipeline) Copy(ctx context.Context, op *compiler.Value, st llb.State) (
} }
func (p *Pipeline) Local(ctx context.Context, op *compiler.Value, st llb.State) (llb.State, error) { func (p *Pipeline) Local(ctx context.Context, op *compiler.Value, st llb.State) (llb.State, error) {
dir, err := op.Lookup("dir").String() id, err := op.Lookup("id").String()
if err != nil { if err != nil {
return st, err return st, err
} }
dir := p.pctx.Directories.Get(plancontext.ContextKey(id))
if dir == nil {
return st, fmt.Errorf("directory %q not found", id)
}
opts := []llb.LocalOption{ opts := []llb.LocalOption{
llb.WithCustomName(p.vertexNamef("Local %s", dir)), llb.WithCustomName(p.vertexNamef("Local %s", dir.Path)),
// Without hint, multiple `llb.Local` operations on the // Without hint, multiple `llb.Local` operations on the
// same path get a different digest. // same path get a different digest.
llb.SessionID(p.s.SessionID()), llb.SessionID(p.s.SessionID()),
llb.SharedKeyHint(dir), llb.SharedKeyHint(dir.Path),
} }
includes, err := op.Lookup("include").List() if len(dir.Include) > 0 {
if err != nil { opts = append(opts, llb.IncludePatterns(dir.Include))
return st, err
}
if len(includes) > 0 {
includePatterns := []string{}
for _, i := range includes {
pattern, err := i.String()
if err != nil {
return st, err
}
includePatterns = append(includePatterns, pattern)
}
opts = append(opts, llb.IncludePatterns(includePatterns))
}
excludes, err := op.Lookup("exclude").List()
if err != nil {
return st, err
} }
// Excludes .dagger directory by default // Excludes .dagger directory by default
excludePatterns := []string{"**/.dagger/"} excludePatterns := []string{"**/.dagger/"}
if len(excludes) > 0 { if len(dir.Exclude) > 0 {
for _, i := range excludes { excludePatterns = dir.Exclude
pattern, err := i.String()
if err != nil {
return st, err
}
excludePatterns = append(excludePatterns, pattern)
}
} }
opts = append(opts, llb.ExcludePatterns(excludePatterns)) opts = append(opts, llb.ExcludePatterns(excludePatterns))
@ -415,13 +396,13 @@ func (p *Pipeline) Local(ctx context.Context, op *compiler.Value, st llb.State)
return st.File( return st.File(
llb.Copy( llb.Copy(
llb.Local( llb.Local(
dir, dir.Path,
opts..., opts...,
), ),
"/", "/",
"/", "/",
), ),
llb.WithCustomName(p.vertexNamef("Local %s [copy]", dir)), llb.WithCustomName(p.vertexNamef("Local %s [copy]", dir.Path)),
), nil ), nil
} }
@ -574,35 +555,15 @@ func (p *Pipeline) mount(ctx context.Context, dest string, mnt *compiler.Value)
return nil, fmt.Errorf("invalid stream %q: not a stream", stream.Path().String()) return nil, fmt.Errorf("invalid stream %q: not a stream", stream.Path().String())
} }
// Unix socket id, err := stream.Lookup("id").String()
unixValue := stream.Lookup("unix") if err != nil {
if unixValue.Exists() { return nil, fmt.Errorf("invalid stream %q: %w", stream.Path().String(), err)
unix, err := unixValue.String()
if err != nil {
return nil, fmt.Errorf("invalid unix path id: %w", err)
}
return llb.AddSSHSocket(
llb.SSHID(fmt.Sprintf("unix=%s", unix)),
llb.SSHSocketTarget(dest),
), nil
} }
// Windows named pipe return llb.AddSSHSocket(
npipeValue := stream.Lookup("npipe") llb.SSHID(id),
if npipeValue.Exists() { llb.SSHSocketTarget(dest),
npipe, err := npipeValue.String() ), nil
if err != nil {
return nil, fmt.Errorf("invalid npipe path id: %w", err)
}
return llb.AddSSHSocket(
llb.SSHID(fmt.Sprintf("npipe=%s", npipe)),
llb.SSHSocketTarget(dest),
), nil
}
return nil, fmt.Errorf("invalid stream %q: not a valid stream", stream.Path().String())
} }
// eg. mount: "/foo": { from: www.source } // eg. mount: "/foo": { from: www.source }
@ -610,7 +571,7 @@ func (p *Pipeline) mount(ctx context.Context, dest string, mnt *compiler.Value)
return nil, fmt.Errorf("invalid mount: should have %s structure", return nil, fmt.Errorf("invalid mount: should have %s structure",
"{from: _, path: string | *\"/\"}") "{from: _, path: string | *\"/\"}")
} }
from := NewPipeline(mnt.Lookup("from"), p.s, p.platform) from := NewPipeline(mnt.Lookup("from"), p.s, p.pctx)
if err := from.Run(ctx); err != nil { if err := from.Run(ctx); err != nil {
return nil, err return nil, err
} }
@ -736,7 +697,7 @@ func unmarshalAnything(data []byte, fn unmarshaller) (interface{}, error) {
} }
// parseStringOrSecret retrieve secret as plain text or retrieve string // parseStringOrSecret retrieve secret as plain text or retrieve string
func parseStringOrSecret(ctx context.Context, ss solver.SecretsStore, v *compiler.Value) (string, error) { func parseStringOrSecret(pctx *plancontext.Context, v *compiler.Value) (string, error) {
// Check if the value is a string, return as is // Check if the value is a string, return as is
if value, err := v.String(); err == nil { if value, err := v.String(); err == nil {
return value, nil return value, nil
@ -747,16 +708,16 @@ func parseStringOrSecret(ctx context.Context, ss solver.SecretsStore, v *compile
if err != nil { if err != nil {
return "", err return "", err
} }
secretBytes, err := ss.GetSecret(ctx, id) secret := pctx.Secrets.Get(plancontext.ContextKey(id))
if err != nil { if secret == nil {
return "", err return "", fmt.Errorf("secret %s not found", id)
} }
return string(secretBytes), nil return secret.PlainText, nil
} }
func (p *Pipeline) Load(ctx context.Context, op *compiler.Value, st llb.State) (llb.State, error) { func (p *Pipeline) Load(ctx context.Context, op *compiler.Value, st llb.State) (llb.State, error) {
// Execute 'from' in a tmp pipeline, and use the resulting fs // Execute 'from' in a tmp pipeline, and use the resulting fs
from := NewPipeline(op.Lookup("from"), p.s, p.platform) from := NewPipeline(op.Lookup("from"), p.s, p.pctx)
if err := from.Run(ctx); err != nil { if err := from.Run(ctx); err != nil {
return st, err return st, err
} }
@ -774,7 +735,7 @@ func (p *Pipeline) DockerLogin(ctx context.Context, op *compiler.Value, st llb.S
// that function // that function
// But currently it's not possible because ECR secret's is a string // But currently it's not possible because ECR secret's is a string
// so we need to handle both options (string & secret) // so we need to handle both options (string & secret)
secretValue, err := parseStringOrSecret(ctx, p.s.GetOptions().SecretsStore, op.Lookup("secret")) secretValue, err := parseStringOrSecret(p.pctx, op.Lookup("secret"))
if err != nil { if err != nil {
return st, err return st, err
} }
@ -813,9 +774,10 @@ func (p *Pipeline) FetchContainer(ctx context.Context, op *compiler.Value, st ll
) )
// Load image metadata and convert to to LLB. // Load image metadata and convert to to LLB.
platform := p.pctx.Platform.Get()
p.image, err = p.s.ResolveImageConfig(ctx, ref.String(), llb.ResolveImageConfigOpt{ p.image, err = p.s.ResolveImageConfig(ctx, ref.String(), llb.ResolveImageConfigOpt{
LogName: p.vertexNamef("load metadata for %s", ref.String()), LogName: p.vertexNamef("load metadata for %s", ref.String()),
Platform: &p.platform, Platform: &platform,
}) })
if err != nil { if err != nil {
return st, err return st, err
@ -869,7 +831,7 @@ func (p *Pipeline) PushContainer(ctx context.Context, op *compiler.Value, st llb
"name": ref.String(), "name": ref.String(),
"push": "true", "push": "true",
}, },
}, p.platform) }, p.pctx.Platform.Get())
if err != nil { if err != nil {
return st, err return st, err
@ -928,7 +890,7 @@ func (p *Pipeline) SaveImage(ctx context.Context, op *compiler.Value, st llb.Sta
Output: func(_ map[string]string) (io.WriteCloser, error) { Output: func(_ map[string]string) (io.WriteCloser, error) {
return pipeW, nil return pipeW, nil
}, },
}, p.platform) }, p.pctx.Platform.Get())
if err != nil { if err != nil {
return st, err return st, err
@ -1088,7 +1050,7 @@ func (p *Pipeline) DockerBuild(ctx context.Context, op *compiler.Value, st llb.S
// docker build context. This can come from another component, so we need to // docker build context. This can come from another component, so we need to
// compute it first. // compute it first.
if dockerContext.Exists() { if dockerContext.Exists() {
from := NewPipeline(op.Lookup("context"), p.s, p.platform) from := NewPipeline(op.Lookup("context"), p.s, p.pctx)
if err := from.Run(ctx); err != nil { if err := from.Run(ctx); err != nil {
return st, err return st, err
} }
@ -1118,7 +1080,7 @@ func (p *Pipeline) DockerBuild(ctx context.Context, op *compiler.Value, st llb.S
} }
} }
opts, err := dockerBuildOpts(ctx, op, p.s.GetOptions().SecretsStore) opts, err := dockerBuildOpts(op, p.pctx)
if err != nil { if err != nil {
return st, err return st, err
} }
@ -1129,7 +1091,7 @@ func (p *Pipeline) DockerBuild(ctx context.Context, op *compiler.Value, st llb.S
// Set platform to configured one if no one is defined // Set platform to configured one if no one is defined
if opts["platform"] == "" { if opts["platform"] == "" {
opts["platform"] = bkplatforms.Format(p.platform) opts["platform"] = bkplatforms.Format(p.pctx.Platform.Get())
} }
req := bkgw.SolveRequest{ req := bkgw.SolveRequest{
@ -1162,7 +1124,7 @@ func (p *Pipeline) DockerBuild(ctx context.Context, op *compiler.Value, st llb.S
return applyImageToState(p.image, st), nil return applyImageToState(p.image, st), nil
} }
func dockerBuildOpts(ctx context.Context, op *compiler.Value, ss solver.SecretsStore) (map[string]string, error) { func dockerBuildOpts(op *compiler.Value, pctx *plancontext.Context) (map[string]string, error) {
opts := map[string]string{} opts := map[string]string{}
if dockerfilePath := op.Lookup("dockerfilePath"); dockerfilePath.Exists() { if dockerfilePath := op.Lookup("dockerfilePath"); dockerfilePath.Exists() {
@ -1205,7 +1167,7 @@ func dockerBuildOpts(ctx context.Context, op *compiler.Value, ss solver.SecretsS
return nil, err return nil, err
} }
for _, buildArg := range fields { for _, buildArg := range fields {
v, err := parseStringOrSecret(ctx, ss, buildArg.Value) v, err := parseStringOrSecret(pctx, buildArg.Value)
if err != nil { if err != nil {
return nil, err return nil, err
} }

48
plancontext/context.go Normal file
View File

@ -0,0 +1,48 @@
package plancontext
import (
"crypto/sha256"
"encoding/json"
"fmt"
)
type ContextKey string
// Context holds the execution context for a plan.
//
// Usage:
// ctx := plancontext.New()
// id := ctx.Secrets.Register("mysecret")
// secret := ctx.Secrets.Get(id)
type Context struct {
Platform *platformContext
Directories *directoryContext
Secrets *secretContext
Services *serviceContext
}
func New() *Context {
return &Context{
Platform: &platformContext{
platform: defaultPlatform,
},
Directories: &directoryContext{
store: make(map[ContextKey]*Directory),
},
Secrets: &secretContext{
store: make(map[ContextKey]*Secret),
},
Services: &serviceContext{
store: make(map[ContextKey]*Service),
},
}
}
func hashID(v interface{}) ContextKey {
data, err := json.Marshal(v)
if err != nil {
panic(err)
}
hash := sha256.Sum256(data)
return ContextKey(fmt.Sprintf("%x", hash))
}

42
plancontext/directory.go Normal file
View File

@ -0,0 +1,42 @@
package plancontext
import "sync"
type Directory struct {
Path string
Include []string
Exclude []string
}
type directoryContext struct {
l sync.RWMutex
store map[ContextKey]*Directory
}
func (c *directoryContext) Register(directory *Directory) ContextKey {
c.l.Lock()
defer c.l.Unlock()
id := hashID(directory)
c.store[id] = directory
return id
}
func (c *directoryContext) Get(id ContextKey) *Directory {
c.l.RLock()
defer c.l.RUnlock()
return c.store[id]
}
func (c *directoryContext) List() []*Directory {
c.l.RLock()
defer c.l.RUnlock()
directories := make([]*Directory, 0, len(c.store))
for _, d := range c.store {
directories = append(directories, d)
}
return directories
}

32
plancontext/platform.go Normal file
View File

@ -0,0 +1,32 @@
package plancontext
import (
"github.com/containerd/containerd/platforms"
specs "github.com/opencontainers/image-spec/specs-go/v1"
)
var (
// Default platform.
// FIXME: This should be auto detected using buildkit
defaultPlatform = specs.Platform{
OS: "linux",
Architecture: "amd64",
}
)
type platformContext struct {
platform specs.Platform
}
func (c *platformContext) Get() specs.Platform {
return c.platform
}
func (c *platformContext) Set(platform string) error {
p, err := platforms.Parse(platform)
if err != nil {
return err
}
c.platform = p
return nil
}

40
plancontext/secret.go Normal file
View File

@ -0,0 +1,40 @@
package plancontext
import "sync"
type secretContext struct {
l sync.RWMutex
store map[ContextKey]*Secret
}
type Secret struct {
PlainText string
}
func (c *secretContext) Register(secret *Secret) ContextKey {
c.l.Lock()
defer c.l.Unlock()
id := hashID(secret.PlainText)
c.store[id] = secret
return id
}
func (c *secretContext) Get(id ContextKey) *Secret {
c.l.RLock()
defer c.l.RUnlock()
return c.store[id]
}
func (c *secretContext) List() []*Secret {
c.l.RLock()
defer c.l.RUnlock()
secrets := make([]*Secret, 0, len(c.store))
for _, s := range c.store {
secrets = append(secrets, s)
}
return secrets
}

29
plancontext/service.go Normal file
View File

@ -0,0 +1,29 @@
package plancontext
import "sync"
type serviceContext struct {
l sync.RWMutex
store map[ContextKey]*Service
}
type Service struct {
Unix string
Npipe string
}
func (c *serviceContext) Register(service *Service) ContextKey {
c.l.Lock()
defer c.l.Unlock()
id := hashID(service)
c.store[id] = service
return id
}
func (c *serviceContext) Get(id ContextKey) *Service {
c.l.RLock()
defer c.l.RUnlock()
return c.store[id]
}

View File

@ -2,55 +2,27 @@ package solver
import ( import (
"context" "context"
"strings"
"github.com/moby/buildkit/session" "github.com/moby/buildkit/session"
"github.com/moby/buildkit/session/secrets" "github.com/moby/buildkit/session/secrets"
"github.com/moby/buildkit/session/secrets/secretsprovider" "github.com/moby/buildkit/session/secrets/secretsprovider"
"github.com/rs/zerolog/log" "github.com/rs/zerolog/log"
"go.dagger.io/dagger/state" "go.dagger.io/dagger/plancontext"
) )
type SecretsStore struct { func NewSecretsStoreProvider(pctx *plancontext.Context) session.Attachable {
Secrets session.Attachable return secretsprovider.NewSecretProvider(&inputStore{pctx})
store *inputStore
}
func (s SecretsStore) GetSecret(ctx context.Context, id string) ([]byte, error) {
return s.store.GetSecret(ctx, id)
}
func NewSecretsStoreProvider(st *state.State) SecretsStore {
store := &inputStore{st}
return SecretsStore{
Secrets: secretsprovider.NewSecretProvider(store),
store: store,
}
} }
type inputStore struct { type inputStore struct {
st *state.State pctx *plancontext.Context
} }
func (s *inputStore) GetSecret(ctx context.Context, id string) ([]byte, error) { func (s *inputStore) GetSecret(ctx context.Context, id string) ([]byte, error) {
lg := log.Ctx(ctx) lg := log.Ctx(ctx)
const secretPrefix = "secret=" secret := s.pctx.Secrets.Get(plancontext.ContextKey(id))
if secret == nil {
if !strings.HasPrefix(id, secretPrefix) {
return nil, secrets.ErrNotFound
}
id = strings.TrimPrefix(id, secretPrefix)
id = strings.Split(id, ";hash=")[0]
input, ok := s.st.Inputs[id]
if !ok {
return nil, secrets.ErrNotFound
}
if input.Secret == nil {
return nil, secrets.ErrNotFound return nil, secrets.ErrNotFound
} }
@ -59,5 +31,5 @@ func (s *inputStore) GetSecret(ctx context.Context, id string) ([]byte, error) {
Str("id", id). Str("id", id).
Msg("injecting secret") Msg("injecting secret")
return []byte(input.Secret.PlainText()), nil return []byte(secret.PlainText), nil
} }

View File

@ -3,24 +3,20 @@ package solver
import ( import (
"context" "context"
"fmt" "fmt"
"strings"
"github.com/moby/buildkit/session" "github.com/moby/buildkit/session"
"github.com/moby/buildkit/session/sshforward" "github.com/moby/buildkit/session/sshforward"
"go.dagger.io/dagger/plancontext"
"google.golang.org/grpc" "google.golang.org/grpc"
"google.golang.org/grpc/metadata" "google.golang.org/grpc/metadata"
) )
const (
unixPrefix = "unix="
npipePrefix = "npipe="
)
type SocketProvider struct { type SocketProvider struct {
pctx *plancontext.Context
} }
func NewDockerSocketProvider() session.Attachable { func NewDockerSocketProvider(pctx *plancontext.Context) session.Attachable {
return &SocketProvider{} return &SocketProvider{pctx}
} }
func (sp *SocketProvider) Register(server *grpc.Server) { func (sp *SocketProvider) Register(server *grpc.Server) {
@ -28,13 +24,6 @@ func (sp *SocketProvider) Register(server *grpc.Server) {
} }
func (sp *SocketProvider) CheckAgent(ctx context.Context, req *sshforward.CheckAgentRequest) (*sshforward.CheckAgentResponse, error) { func (sp *SocketProvider) CheckAgent(ctx context.Context, req *sshforward.CheckAgentRequest) (*sshforward.CheckAgentResponse, error) {
id := sshforward.DefaultID
if req.ID != "" {
id = req.ID
}
if !strings.HasPrefix(id, unixPrefix) && !strings.HasPrefix(id, npipePrefix) {
return &sshforward.CheckAgentResponse{}, fmt.Errorf("invalid socket forward key %s", id)
}
return &sshforward.CheckAgentResponse{}, nil return &sshforward.CheckAgentResponse{}, nil
} }
@ -47,7 +36,12 @@ func (sp *SocketProvider) ForwardAgent(stream sshforward.SSH_ForwardAgentServer)
id = v[0] id = v[0]
} }
conn, err := dialStream(id) service := sp.pctx.Services.Get(plancontext.ContextKey(id))
if service == nil {
return fmt.Errorf("invalid socket id %q", id)
}
conn, err := dialService(service)
if err != nil { if err != nil {
return fmt.Errorf("failed to connect to %s: %w", id, err) return fmt.Errorf("failed to connect to %s: %w", id, err)
} }

View File

@ -4,17 +4,17 @@
package solver package solver
import ( import (
"fmt" "errors"
"net" "net"
"strings"
"time" "time"
"go.dagger.io/dagger/plancontext"
) )
func dialStream(id string) (net.Conn, error) { func dialService(service *plancontext.Service) (net.Conn, error) {
if !strings.HasPrefix(id, unixPrefix) { if service.Unix == "" {
return nil, fmt.Errorf("invalid socket forward key %s", id) return nil, errors.New("unsupported socket type")
} }
id = strings.TrimPrefix(id, unixPrefix) return net.DialTimeout("unix", service.Unix, time.Second)
return net.DialTimeout("unix", id, time.Second)
} }

View File

@ -4,20 +4,19 @@
package solver package solver
import ( import (
"fmt" "errors"
"net" "net"
"strings"
"time" "time"
"github.com/Microsoft/go-winio" "github.com/Microsoft/go-winio"
"go.dagger.io/dagger/plancontext"
) )
func dialStream(id string) (net.Conn, error) { func dialService(service *plancontext.Service) (net.Conn, error) {
if !strings.HasPrefix(id, npipePrefix) { if service.Npipe == "" {
return nil, fmt.Errorf("invalid socket forward key %s", id) return nil, errors.New("unsupported socket type")
} }
id = strings.TrimPrefix(id, npipePrefix)
dur := time.Second dur := time.Second
return winio.DialPipe(id, &dur) return winio.DialPipe(service.Npipe, &dur)
} }

View File

@ -18,6 +18,7 @@ import (
"github.com/opencontainers/go-digest" "github.com/opencontainers/go-digest"
specs "github.com/opencontainers/image-spec/specs-go/v1" specs "github.com/opencontainers/image-spec/specs-go/v1"
"github.com/rs/zerolog/log" "github.com/rs/zerolog/log"
"go.dagger.io/dagger/plancontext"
) )
type Solver struct { type Solver struct {
@ -27,12 +28,12 @@ type Solver struct {
} }
type Opts struct { type Opts struct {
Control *bk.Client Control *bk.Client
Gateway bkgw.Client Gateway bkgw.Client
Events chan *bk.SolveStatus Events chan *bk.SolveStatus
Auth *RegistryAuthProvider Context *plancontext.Context
SecretsStore SecretsStore Auth *RegistryAuthProvider
NoCache bool NoCache bool
} }
func New(opts Opts) Solver { func New(opts Opts) Solver {
@ -194,8 +195,8 @@ func (s Solver) Export(ctx context.Context, st llb.State, img *dockerfile2llb.Im
Exports: []bk.ExportEntry{output}, Exports: []bk.ExportEntry{output},
Session: []session.Attachable{ Session: []session.Attachable{
s.opts.Auth, s.opts.Auth,
s.opts.SecretsStore.Secrets, NewSecretsStoreProvider(s.opts.Context),
NewDockerSocketProvider(), NewDockerSocketProvider(s.opts.Context),
}, },
} }

View File

@ -1,8 +1,6 @@
package state package state
import ( import (
"crypto/sha256"
"encoding/json"
"fmt" "fmt"
"io/ioutil" "io/ioutil"
"os" "os"
@ -13,6 +11,7 @@ import (
"cuelang.org/go/cue" "cuelang.org/go/cue"
"go.dagger.io/dagger/compiler" "go.dagger.io/dagger/compiler"
"go.dagger.io/dagger/plancontext"
) )
// An input is a value or artifact supplied by the user. // An input is a value or artifact supplied by the user.
@ -31,7 +30,6 @@ import (
type Input struct { type Input struct {
Dir *dirInput `yaml:"dir,omitempty"` Dir *dirInput `yaml:"dir,omitempty"`
Git *gitInput `yaml:"git,omitempty"` Git *gitInput `yaml:"git,omitempty"`
Docker *dockerInput `yaml:"docker,omitempty"`
Secret *secretInput `yaml:"secret,omitempty"` Secret *secretInput `yaml:"secret,omitempty"`
Text *textInput `yaml:"text,omitempty"` Text *textInput `yaml:"text,omitempty"`
JSON *jsonInput `yaml:"json,omitempty"` JSON *jsonInput `yaml:"json,omitempty"`
@ -41,28 +39,26 @@ type Input struct {
Socket *socketInput `yaml:"socket,omitempty"` Socket *socketInput `yaml:"socket,omitempty"`
} }
func (i Input) Compile(key string, state *State) (*compiler.Value, error) { func (i Input) Compile(state *State) (*compiler.Value, error) {
switch { switch {
case i.Dir != nil: case i.Dir != nil:
return i.Dir.Compile(key, state) return i.Dir.Compile(state)
case i.Git != nil: case i.Git != nil:
return i.Git.Compile(key, state) return i.Git.Compile(state)
case i.Docker != nil:
return i.Docker.Compile(key, state)
case i.Text != nil: case i.Text != nil:
return i.Text.Compile(key, state) return i.Text.Compile(state)
case i.Secret != nil: case i.Secret != nil:
return i.Secret.Compile(key, state) return i.Secret.Compile(state)
case i.JSON != nil: case i.JSON != nil:
return i.JSON.Compile(key, state) return i.JSON.Compile(state)
case i.YAML != nil: case i.YAML != nil:
return i.YAML.Compile(key, state) return i.YAML.Compile(state)
case i.File != nil: case i.File != nil:
return i.File.Compile(key, state) return i.File.Compile(state)
case i.Bool != nil: case i.Bool != nil:
return i.Bool.Compile(key, state) return i.Bool.Compile(state)
case i.Socket != nil: case i.Socket != nil:
return i.Socket.Compile(key, state) return i.Socket.Compile(state)
default: default:
return nil, fmt.Errorf("input has not been set") return nil, fmt.Errorf("input has not been set")
} }
@ -85,28 +81,7 @@ type dirInput struct {
Exclude []string `yaml:"exclude,omitempty"` Exclude []string `yaml:"exclude,omitempty"`
} }
func (dir dirInput) Compile(_ string, state *State) (*compiler.Value, error) { func (dir dirInput) Compile(state *State) (*compiler.Value, error) {
// FIXME: serialize an intermediate struct, instead of generating cue source
// json.Marshal([]string{}) returns []byte("null"), which wreaks havoc
// in Cue because `null` is not a `[...string]`
includeLLB := []byte("[]")
if len(dir.Include) > 0 {
var err error
includeLLB, err = json.Marshal(dir.Include)
if err != nil {
return nil, err
}
}
excludeLLB := []byte("[]")
if len(dir.Exclude) > 0 {
var err error
excludeLLB, err = json.Marshal(dir.Exclude)
if err != nil {
return nil, err
}
}
p := dir.Path p := dir.Path
if !filepath.IsAbs(p) { if !filepath.IsAbs(p) {
p = filepath.Clean(path.Join(state.Project, dir.Path)) p = filepath.Clean(path.Join(state.Project, dir.Path))
@ -119,16 +94,15 @@ func (dir dirInput) Compile(_ string, state *State) (*compiler.Value, error) {
return nil, fmt.Errorf("%q dir doesn't exist", dir.Path) return nil, fmt.Errorf("%q dir doesn't exist", dir.Path)
} }
dirPath, err := json.Marshal(p) id := state.Context.Directories.Register(&plancontext.Directory{
if err != nil { Path: p,
return nil, err Include: dir.Include,
} Exclude: dir.Exclude,
})
llb := fmt.Sprintf( llb := fmt.Sprintf(
`#up: [{do:"local",dir:%s, include:%s, exclude:%s}]`, `#up: [{do:"local", id: "%s"}]`,
dirPath, id,
includeLLB,
excludeLLB,
) )
return compiler.Compile("", llb) return compiler.Compile("", llb)
} }
@ -150,7 +124,7 @@ func GitInput(remote, ref, dir string) Input {
} }
} }
func (git gitInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (git gitInput) Compile(_ *State) (*compiler.Value, error) {
ref := "HEAD" ref := "HEAD"
if git.Ref != "" { if git.Ref != "" {
ref = git.Ref ref = git.Ref
@ -169,23 +143,6 @@ func (git gitInput) Compile(_ string, _ *State) (*compiler.Value, error) {
)) ))
} }
// An input artifact loaded from a docker container
func DockerInput(ref string) Input {
return Input{
Docker: &dockerInput{
Ref: ref,
},
}
}
type dockerInput struct {
Ref string `yaml:"ref,omitempty"`
}
func (i dockerInput) Compile(_ string, _ *State) (*compiler.Value, error) {
panic("NOT IMPLEMENTED")
}
// An input value encoded as text // An input value encoded as text
func TextInput(data string) Input { func TextInput(data string) Input {
i := textInput(data) i := textInput(data)
@ -196,7 +153,7 @@ func TextInput(data string) Input {
type textInput string type textInput string
func (i textInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (i textInput) Compile(_ *State) (*compiler.Value, error) {
return compiler.Compile("", fmt.Sprintf("%q", i)) return compiler.Compile("", fmt.Sprintf("%q", i))
} }
@ -210,11 +167,11 @@ func SecretInput(data string) Input {
type secretInput string type secretInput string
func (i secretInput) Compile(key string, _ *State) (*compiler.Value, error) { func (i secretInput) Compile(st *State) (*compiler.Value, error) {
hash := sha256.New() id := st.Context.Secrets.Register(&plancontext.Secret{
hash.Write([]byte(key)) PlainText: i.PlainText(),
checksum := hash.Sum([]byte(i.PlainText())) })
secretValue := fmt.Sprintf(`{id:"secret=%s;hash=%x"}`, key, checksum) secretValue := fmt.Sprintf(`{id: %q}`, id)
return compiler.Compile("", secretValue) return compiler.Compile("", secretValue)
} }
@ -232,7 +189,7 @@ func BoolInput(data string) Input {
type boolInput string type boolInput string
func (i boolInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (i boolInput) Compile(_ *State) (*compiler.Value, error) {
s := map[boolInput]struct{}{ s := map[boolInput]struct{}{
"true": {}, "true": {},
"false": {}, "false": {},
@ -253,7 +210,7 @@ func JSONInput(data string) Input {
type jsonInput string type jsonInput string
func (i jsonInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (i jsonInput) Compile(_ *State) (*compiler.Value, error) {
return compiler.DecodeJSON("", []byte(i)) return compiler.DecodeJSON("", []byte(i))
} }
@ -267,7 +224,7 @@ func YAMLInput(data string) Input {
type yamlInput string type yamlInput string
func (i yamlInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (i yamlInput) Compile(_ *State) (*compiler.Value, error) {
return compiler.DecodeYAML("", []byte(i)) return compiler.DecodeYAML("", []byte(i))
} }
@ -283,7 +240,7 @@ type fileInput struct {
Path string `yaml:"path,omitempty"` Path string `yaml:"path,omitempty"`
} }
func (i fileInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (i fileInput) Compile(_ *State) (*compiler.Value, error) {
data, err := ioutil.ReadFile(i.Path) data, err := ioutil.ReadFile(i.Path)
if err != nil { if err != nil {
return nil, err return nil, err
@ -316,11 +273,11 @@ type socketInput struct {
Npipe string `json:"npipe,omitempty" yaml:"npipe,omitempty"` Npipe string `json:"npipe,omitempty" yaml:"npipe,omitempty"`
} }
func (i socketInput) Compile(_ string, _ *State) (*compiler.Value, error) { func (i socketInput) Compile(st *State) (*compiler.Value, error) {
socketValue, err := json.Marshal(i) id := st.Context.Services.Register(&plancontext.Service{
if err != nil { Unix: i.Unix,
return nil, err Npipe: i.Npipe,
} })
socketValue := fmt.Sprintf(`{id: %q}`, id)
return compiler.Compile("", string(socketValue)) return compiler.Compile("", socketValue)
} }

View File

@ -12,6 +12,7 @@ import (
"github.com/rs/zerolog/log" "github.com/rs/zerolog/log"
"go.dagger.io/dagger/keychain" "go.dagger.io/dagger/keychain"
"go.dagger.io/dagger/plancontext"
"go.dagger.io/dagger/stdlib" "go.dagger.io/dagger/stdlib"
"gopkg.in/yaml.v3" "gopkg.in/yaml.v3"
) )
@ -168,6 +169,12 @@ func (w *Project) Get(ctx context.Context, name string) (*State, error) {
if err := yaml.Unmarshal(manifest, &st); err != nil { if err := yaml.Unmarshal(manifest, &st); err != nil {
return nil, err return nil, err
} }
st.Context = plancontext.New()
if platform := st.Platform; platform != "" {
if err := st.Context.Platform.Set(platform); err != nil {
return nil, err
}
}
st.Path = envPath st.Path = envPath
// FIXME: Backward compat: Support for old-style `.dagger/env/<name>/plan` // FIXME: Backward compat: Support for old-style `.dagger/env/<name>/plan`
if st.Plan.Module == "" { if st.Plan.Module == "" {
@ -258,6 +265,8 @@ func (w *Project) Create(ctx context.Context, name string, plan Plan, platform s
manifestPath := path.Join(envPath, manifestFile) manifestPath := path.Join(envPath, manifestFile)
st := &State{ st := &State{
Context: plancontext.New(),
Path: envPath, Path: envPath,
Project: w.Path, Project: w.Path,
Plan: Plan{ Plan: Plan{

View File

@ -6,10 +6,15 @@ import (
"cuelang.org/go/cue" "cuelang.org/go/cue"
"go.dagger.io/dagger/compiler" "go.dagger.io/dagger/compiler"
"go.dagger.io/dagger/plancontext"
) )
// Contents of an environment serialized to a file // Contents of an environment serialized to a file
type State struct { type State struct {
// Plan Context.
// FIXME: this is used as a bridge and is temporary.
Context *plancontext.Context `yaml:"-"`
// State path // State path
Path string `yaml:"-"` Path string `yaml:"-"`
@ -67,7 +72,7 @@ func (s *State) CompileInputs() (*compiler.Value, error) {
// Prepare inputs // Prepare inputs
for key, input := range s.Inputs { for key, input := range s.Inputs {
i, err := input.Compile(key, s) i, err := input.Compile(s)
if err != nil { if err != nil {
return nil, err return nil, err
} }

View File

@ -18,13 +18,7 @@ import (
#Stream: { #Stream: {
@dagger(stream) @dagger(stream)
{ id: string
// Unix Socket
unix: string
} | {
// Windows Named Pipe
npipe: string
}
} }
// Secret value // Secret value

View File

@ -20,10 +20,8 @@ package op
} }
#Local: { #Local: {
do: "local" do: "local"
dir: string id: string
include: [...string]
exclude: [...string]
} }
// FIXME: bring back load (more efficient than copy) // FIXME: bring back load (more efficient than copy)

View File

@ -160,6 +160,9 @@ import (
string string
#up: [ #up: [
// HACK: force a dependency with `load`
op.#Load & {from: load},
op.#Load & {from: save}, op.#Load & {from: save},
op.#Export & { op.#Export & {

View File

@ -145,7 +145,6 @@ setup() {
# Make sure the secret doesn't show in dagger query # Make sure the secret doesn't show in dagger query
run "$DAGGER" query mySecret.id -f text run "$DAGGER" query mySecret.id -f text
assert_success assert_success
assert_output --partial "secret=mySecret;hash="
} }
@test "core: stream" { @test "core: stream" {