Skip to content

Commit

Permalink
Split up Environment into BuildDirectory and Runner
Browse files Browse the repository at this point in the history
When support for multiple runners for a single worker as added, we ran
into a pretty serious design flaw: the cardinality between the number of
build directories and runners had changed: 1:1 -> 1:*. Back then we sort
of ignored this issue, which led interesting bugs, such as cleaning of
the build directory not working properly in multi-runner setups. We
eventually worked around this by disabling this feature. Some TODOs in
cmd/bb_worker/main.go remained.

The goal of this change is to finally tackle that problem by basically
kicking out the Environment interface, which caused the 1:1 coupling. It
introduces separate BuildDirectory and Runner types. LocalBuildExecutor
now takes individual copies of each.

Because of this decoupling, we can also fix #20. Builds are no longer
performed in the build directory itself. Instead, they are run inside a
subdirectory called "root". This makes it possible to use the top level
directory for other purposes (e.g., storing stdout and stderr logs).

It also partially addresses #23. The runner protocol has been extended
to explicitly pass on the input root directory, so that a runner could
chroot into it.
  • Loading branch information
EdSchouten committed Apr 2, 2020
1 parent e44cc90 commit 94b9fb6
Show file tree
Hide file tree
Showing 40 changed files with 1,114 additions and 1,045 deletions.
2 changes: 1 addition & 1 deletion cmd/bb_runner/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -8,9 +8,9 @@ go_library(
importpath = "github.com/buildbarn/bb-remote-execution/cmd/bb_runner",
visibility = ["//visibility:private"],
deps = [
"//pkg/environment:go_default_library",
"//pkg/proto/configuration/bb_runner:go_default_library",
"//pkg/proto/runner:go_default_library",
"//pkg/runner:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/filesystem:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/global:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/grpc:go_default_library",
Expand Down
17 changes: 9 additions & 8 deletions cmd/bb_runner/main.go
Original file line number Diff line number Diff line change
Expand Up @@ -4,9 +4,9 @@ import (
"log"
"os"

"github.com/buildbarn/bb-remote-execution/pkg/environment"
"github.com/buildbarn/bb-remote-execution/pkg/proto/configuration/bb_runner"
"github.com/buildbarn/bb-remote-execution/pkg/proto/runner"
runner_pb "github.com/buildbarn/bb-remote-execution/pkg/proto/runner"
"github.com/buildbarn/bb-remote-execution/pkg/runner"
"github.com/buildbarn/bb-storage/pkg/filesystem"
"github.com/buildbarn/bb-storage/pkg/global"
bb_grpc "github.com/buildbarn/bb-storage/pkg/grpc"
Expand All @@ -32,24 +32,25 @@ func main() {
log.Fatal("Failed to open build directory: ", err)
}

env := environment.NewLocalExecutionEnvironment(buildDirectory, configuration.BuildDirectoryPath)
r := runner.NewLocalRunner(
buildDirectory,
configuration.BuildDirectoryPath)

// When temporary directories need cleaning prior to executing a build
// action, attach a series of TempDirectoryCleaningManagers.
m := environment.NewSingletonManager(env)
// action, attach a series of TemporaryDirectoryCleaningRunners.
for _, d := range configuration.TemporaryDirectories {
directory, err := filesystem.NewLocalDirectory(d)
if err != nil {
log.Fatalf("Failed to open temporary directory %#v: %s", d, err)
}
m = environment.NewTempDirectoryCleaningManager(m, directory)
r = runner.NewTemporaryDirectoryCleaningRunner(r, directory, d)
}
runnerServer := environment.NewRunnerServer(environment.NewConcurrentManager(m))

log.Fatal(
"gRPC server failure: ",
bb_grpc.NewGRPCServersFromConfigurationAndServe(
configuration.GrpcServers,
func(s *grpc.Server) {
runner.RegisterRunnerServer(s, runnerServer)
runner_pb.RegisterRunnerServer(s, runner.NewRunnerServer(r))
}))
}
3 changes: 2 additions & 1 deletion cmd/bb_worker/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -11,11 +11,12 @@ go_library(
"//pkg/blobstore:go_default_library",
"//pkg/builder:go_default_library",
"//pkg/cas:go_default_library",
"//pkg/environment:go_default_library",
"//pkg/filesystem:go_default_library",
"//pkg/proto/configuration/bb_worker:go_default_library",
"//pkg/proto/remoteworker:go_default_library",
"//pkg/proto/runner:go_default_library",
"//pkg/runner:go_default_library",
"//pkg/sync:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/blobstore:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/blobstore/configuration:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/cas:go_default_library",
Expand Down
68 changes: 27 additions & 41 deletions cmd/bb_worker/main.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,11 +15,12 @@ import (
re_blobstore "github.com/buildbarn/bb-remote-execution/pkg/blobstore"
"github.com/buildbarn/bb-remote-execution/pkg/builder"
re_cas "github.com/buildbarn/bb-remote-execution/pkg/cas"
"github.com/buildbarn/bb-remote-execution/pkg/environment"
re_filesystem "github.com/buildbarn/bb-remote-execution/pkg/filesystem"
"github.com/buildbarn/bb-remote-execution/pkg/proto/configuration/bb_worker"
"github.com/buildbarn/bb-remote-execution/pkg/proto/remoteworker"
"github.com/buildbarn/bb-remote-execution/pkg/proto/runner"
runner_pb "github.com/buildbarn/bb-remote-execution/pkg/proto/runner"
"github.com/buildbarn/bb-remote-execution/pkg/runner"
"github.com/buildbarn/bb-remote-execution/pkg/sync"
"github.com/buildbarn/bb-storage/pkg/blobstore"
blobstore_configuration "github.com/buildbarn/bb-storage/pkg/blobstore/configuration"
"github.com/buildbarn/bb-storage/pkg/cas"
Expand Down Expand Up @@ -82,7 +83,7 @@ func main() {
log.Fatal("Failed to create blob access: ", err)
}

var buildDirectory filesystem.Directory
var naiveBuildDirectory filesystem.DirectoryCloser
switch buildDirectoryConfigurationVariant := configuration.BuildDirectory.(type) {
case *bb_worker.ApplicationConfiguration_LocalBuildDirectory:
// To ease privilege separation, clear the umask. This
Expand All @@ -93,16 +94,10 @@ func main() {

// Directory where actual builds take place.
buildDirectoryConfiguration := buildDirectoryConfigurationVariant.LocalBuildDirectory
buildDirectory, err = filesystem.NewLocalDirectory(buildDirectoryConfiguration.BuildDirectoryPath)
naiveBuildDirectory, err = filesystem.NewLocalDirectory(buildDirectoryConfiguration.BuildDirectoryPath)
if err != nil {
log.Fatal("Failed to open build directory: ", err)
}
// TODO: This may be removed when the
// CleanBuildDirectoryManager is enabled unconditionally
// once again.
if err := buildDirectory.RemoveAllChildren(); err != nil {
log.Fatal("Failed to clean build directory on startup: ", err)
}
default:
log.Fatal("No build directory specified")
}
Expand Down Expand Up @@ -143,6 +138,8 @@ func main() {
eviction.NewMetricsSet(evictionSet, "HardlinkingContentAddressableStorage"))
}

var buildDirectoryInitializer sync.Initializer
var sharedBuildDirectoryNextParallelActionID uint64
if len(configuration.Runners) == 0 {
log.Fatal("Cannot start worker without any runners")
}
Expand Down Expand Up @@ -173,7 +170,7 @@ func main() {
}

// Wait for the runner process to come online.
runnerClient := runner.NewRunnerClient(runnerConnection)
runnerClient := runner_pb.NewRunnerClient(runnerConnection)
for {
_, err := runnerClient.CheckReadiness(context.Background(), &empty.Empty{})
if err == nil {
Expand All @@ -183,32 +180,6 @@ func main() {
time.Sleep(3 * time.Second)
}

// Build environment capable of executing one action at a time.
// The build takes place in the root of the build directory.
environmentManager := environment.NewSingletonManager(
environment.NewRemoteExecutionEnvironment(runnerConnection, buildDirectory))

// Clean the build directory every time when going from
// fully idle to executing one action.
// TODO: Also enable this feature when multiple runners
// are configured. Being able to support this requires a
// decomposition of environment.Environment into two
// separate interfaces: one for managing the build
// directory and one for running commands.
if len(configuration.Runners) == 1 {
environmentManager = environment.NewCleanBuildDirectoryManager(environmentManager)
}

// Create a per-action subdirectory in the build directory named
// after the action digest, so that multiple actions may be run
// concurrently within the same environment.
// TODO(edsch): It might make sense to disable this if
// concurrency is disabled to improve action cache hit rate, but
// only if there are no other workers in the same cluster that
// have concurrency enabled.
environmentManager = environment.NewActionDigestSubdirectoryManager(
environment.NewConcurrentManager(environmentManager))

for threadID := uint64(0); threadID < runnerConfiguration.Concurrency; threadID++ {
go func(runnerConfiguration *bb_worker.RunnerConfiguration, threadID uint64) {
// Per-worker separate writer of the Content
Expand All @@ -227,13 +198,28 @@ func main() {
contentAddressableStorageWriter,
int(configuration.MaximumMessageSizeBytes)))

var inputRootPopulator builder.InputRootPopulator
var buildDirectory builder.BuildDirectory
switch configuration.BuildDirectory.(type) {
case *bb_worker.ApplicationConfiguration_LocalBuildDirectory:
inputRootPopulator = builder.NewNaiveInputRootPopulator(
buildDirectory = builder.NewNaiveBuildDirectory(
naiveBuildDirectory,
contentAddressableStorage)
}

// Create a per-action subdirectory in
// the build directory named after the
// action digest, so that multiple
// actions may be run concurrently.
//
// Also clean the build directory every
// time when going from fully idle to
// executing one action.
buildDirectoryCreator := builder.NewSharedBuildDirectoryCreator(
builder.NewCleanBuildDirectoryCreator(
builder.NewRootBuildDirectoryCreator(buildDirectory),
&buildDirectoryInitializer),
&sharedBuildDirectoryNextParallelActionID)

workerID := map[string]string{}
if runnerConfiguration.Concurrency > 1 {
workerID["thread"] = fmt.Sprintf("%0*d", concurrencyLength, threadID)
Expand All @@ -254,8 +240,8 @@ func main() {
builder.NewStorageFlushingBuildExecutor(
builder.NewLocalBuildExecutor(
contentAddressableStorage,
environmentManager,
inputRootPopulator,
buildDirectoryCreator,
runner.NewRemoteRunner(runnerConnection),
clock.SystemClock,
defaultExecutionTimeout,
maximumExecutionTimeout),
Expand Down
26 changes: 11 additions & 15 deletions internal/mock/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -15,8 +15,9 @@ gomock(
name = "builder",
out = "builder.go",
interfaces = [
"BuildDirectory",
"BuildDirectoryCreator",
"BuildExecutor",
"InputRootPopulator",
"StorageFlusher",
],
library = "//pkg/builder:go_default_library",
Expand All @@ -42,18 +43,6 @@ gomock(
package = "mock",
)

gomock(
name = "environment",
out = "environment.go",
interfaces = [
"Environment",
"ManagedEnvironment",
"Manager",
],
library = "//pkg/environment:go_default_library",
package = "mock",
)

gomock(
name = "filesystem",
out = "filesystem.go",
Expand Down Expand Up @@ -87,6 +76,14 @@ gomock(
package = "mock",
)

gomock(
name = "runner",
out = "runner.go",
interfaces = ["Runner"],
library = "//pkg/runner:go_default_library",
package = "mock",
)

gomock(
name = "sharding",
out = "sharding.go",
Expand Down Expand Up @@ -132,10 +129,10 @@ go_library(
":builder.go",
":cas.go",
":clock.go",
":environment.go",
":filesystem.go",
":filesystem_re.go",
":remoteexecution.go",
":runner.go",
":sharding.go",
":storage_builder.go",
":storage_util.go",
Expand All @@ -145,7 +142,6 @@ go_library(
visibility = ["//:__subpackages__"],
deps = [
"//pkg/builder:go_default_library",
"//pkg/environment:go_default_library",
"//pkg/filesystem:go_default_library",
"//pkg/proto/remoteworker:go_default_library",
"//pkg/proto/runner:go_default_library",
Expand Down
17 changes: 13 additions & 4 deletions pkg/builder/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -4,27 +4,32 @@ go_library(
name = "go_default_library",
srcs = [
"build_client.go",
"build_directory.go",
"build_directory_creator.go",
"build_executor.go",
"build_queue_state_provider.go",
"caching_build_executor.go",
"clean_build_directory_creator.go",
"file_pool_stats_build_executor.go",
"in_memory_build_queue.go",
"input_root_populator.go",
"local_build_executor.go",
"logging_build_executor.go",
"metrics_build_executor.go",
"naive_input_root_populator.go",
"naive_build_directory.go",
"root_build_directory_creator.go",
"shared_build_directory_creator.go",
"storage_flushing_build_executor.go",
"timestamped_build_executor.go",
],
importpath = "github.com/buildbarn/bb-remote-execution/pkg/builder",
visibility = ["//visibility:public"],
deps = [
"//pkg/environment:go_default_library",
"//pkg/filesystem:go_default_library",
"//pkg/proto/remoteworker:go_default_library",
"//pkg/proto/resourceusage:go_default_library",
"//pkg/proto/runner:go_default_library",
"//pkg/runner:go_default_library",
"//pkg/sync:go_default_library",
"//pkg/util:go_default_library",
"@com_github_bazelbuild_remote_apis//build/bazel/remote/execution/v2:go_default_library",
"@com_github_bazelbuild_remote_apis//build/bazel/semver:go_default_library",
Expand Down Expand Up @@ -55,10 +60,13 @@ go_test(
name = "go_default_test",
srcs = [
"caching_build_executor_test.go",
"clean_build_directory_creator_test.go",
"file_pool_stats_build_executor_test.go",
"in_memory_build_queue_test.go",
"local_build_executor_test.go",
"naive_input_root_populator_test.go",
"naive_build_directory_test.go",
"root_build_directory_creator_test.go",
"shared_build_directory_creator_test.go",
"storage_flushing_build_executor_test.go",
"timestamped_build_executor_test.go",
],
Expand All @@ -69,6 +77,7 @@ go_test(
"//pkg/proto/remoteworker:go_default_library",
"//pkg/proto/resourceusage:go_default_library",
"//pkg/proto/runner:go_default_library",
"//pkg/sync:go_default_library",
"@com_github_bazelbuild_remote_apis//build/bazel/remote/execution/v2:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/blobstore/buffer:go_default_library",
"@com_github_buildbarn_bb_storage//pkg/builder:go_default_library",
Expand Down
32 changes: 32 additions & 0 deletions pkg/builder/build_directory.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,32 @@
package builder

import (
"context"

re_filesystem "github.com/buildbarn/bb-remote-execution/pkg/filesystem"
"github.com/buildbarn/bb-storage/pkg/digest"
"github.com/buildbarn/bb-storage/pkg/filesystem"
)

// BuildDirectory is a filesystem.Directory that may be used for the
// purpose of running build actions. BuildDirectory has a couple of
// special operations that implementations may use to run actions in a
// more efficient and manageable way.
type BuildDirectory interface {
filesystem.DirectoryCloser

// Identical to EnterDirectory(), except that it returns a
// BuildDirectory object.
EnterBuildDirectory(name string) (BuildDirectory, error)

// Installs a set of hooks into the directory to intercept I/O
// operations. The FilePool may be used to allocate storage
// space. Implementations of BuildDirectory are free to let this
// be a no-op, with the disadvantage that they cannot apply
// resource limits or provide rich I/O error messages.
InstallHooks(filePool re_filesystem.FilePool)

// Recursively merges the contents of a Directory stored in the
// Content Addressable Storage into a local directory.
MergeDirectoryContents(ctx context.Context, digest digest.Digest) error
}
11 changes: 11 additions & 0 deletions pkg/builder/build_directory_creator.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,11 @@
package builder

import (
"github.com/buildbarn/bb-storage/pkg/digest"
)

// BuildDirectoryCreator is used by LocalBuildExecutor to obtain build
// directories in which build actions are executed.
type BuildDirectoryCreator interface {
GetBuildDirectory(actionDigest digest.Digest, mayRunInParallel bool) (BuildDirectory, string, error)
}
Loading

0 comments on commit 94b9fb6

Please sign in to comment.