Merge branch 'master' into 8554-trash-untrash-unix-volume
[arvados.git] / services / crunch-run / crunchrun.go
index d197fa95f73765b194a07fa363cdac8c402b51fc..01edb0a516fadd33c175df030c7c4c330b2985ba 100644 (file)
@@ -18,6 +18,7 @@ import (
        "strings"
        "sync"
        "syscall"
+       "time"
 )
 
 // IArvadosClient is the minimal Arvados API methods used by crunch-run.
@@ -69,6 +70,10 @@ type ContainerRecord struct {
 // NewLogWriter is a factory function to create a new log writer.
 type NewLogWriter func(name string) io.WriteCloser
 
+type RunArvMount func([]string) (*exec.Cmd, error)
+
+type MkTempDir func(string, string) (string, error)
+
 // ThinDockerClient is the minimal Docker client interface used by crunch-run.
 type ThinDockerClient interface {
        StopContainer(id string, timeout int) error
@@ -76,7 +81,7 @@ type ThinDockerClient interface {
        LoadImage(reader io.Reader) error
        CreateContainer(config *dockerclient.ContainerConfig, name string, authConfig *dockerclient.AuthConfig) (string, error)
        StartContainer(id string, config *dockerclient.HostConfig) error
-       ContainerLogs(id string, options *dockerclient.LogOptions) (io.ReadCloser, error)
+       AttachContainer(id string, options *dockerclient.AttachOptions) (io.ReadCloser, error)
        Wait(id string) <-chan dockerclient.WaitResult
        RemoveImage(name string, force bool) ([]*dockerclient.ImageDelete, error)
 }
@@ -98,20 +103,24 @@ type ContainerRunner struct {
        Stderr        *ThrottledLogger
        LogCollection *CollectionWriter
        LogsPDH       *string
-       ArvMount      *exec.Cmd
-       ArvMountPoint string
-       HostOutputDir string
-       Binds         []string
-       OutputPDH     *string
-       CancelLock    sync.Mutex
-       Cancelled     bool
-       SigChan       chan os.Signal
-       finalState    string
+       RunArvMount
+       MkTempDir
+       ArvMount       *exec.Cmd
+       ArvMountPoint  string
+       HostOutputDir  string
+       CleanupTempDir []string
+       Binds          []string
+       OutputPDH      *string
+       CancelLock     sync.Mutex
+       Cancelled      bool
+       SigChan        chan os.Signal
+       ArvMountExit   chan error
+       finalState     string
 }
 
 // SetupSignals sets up signal handling to gracefully terminate the underlying
 // Docker container and update state when receiving a TERM, INT or QUIT signal.
-func (runner *ContainerRunner) SetupSignals() error {
+func (runner *ContainerRunner) SetupSignals() {
        runner.SigChan = make(chan os.Signal, 1)
        signal.Notify(runner.SigChan, syscall.SIGTERM)
        signal.Notify(runner.SigChan, syscall.SIGINT)
@@ -129,8 +138,6 @@ func (runner *ContainerRunner) SetupSignals() error {
                        }
                }
        }(runner.SigChan)
-
-       return nil
 }
 
 // LoadImage determines the docker image id from the container record and
@@ -180,16 +187,63 @@ func (runner *ContainerRunner) LoadImage() (err error) {
        return nil
 }
 
+func (runner *ContainerRunner) ArvMountCmd(arvMountCmd []string) (c *exec.Cmd, err error) {
+       c = exec.Command("arv-mount", arvMountCmd...)
+       nt := NewThrottledLogger(runner.NewLogWriter("arv-mount"))
+       c.Stdout = nt
+       c.Stderr = nt
+
+       err = c.Start()
+       if err != nil {
+               return nil, err
+       }
+
+       statReadme := make(chan bool)
+       runner.ArvMountExit = make(chan error)
+
+       keepStatting := true
+       go func() {
+               for keepStatting {
+                       time.Sleep(100 * time.Millisecond)
+                       _, err = os.Stat(fmt.Sprintf("%s/by_id/README", runner.ArvMountPoint))
+                       if err == nil {
+                               keepStatting = false
+                               statReadme <- true
+                       }
+               }
+               close(statReadme)
+       }()
+
+       go func() {
+               runner.ArvMountExit <- c.Wait()
+               close(runner.ArvMountExit)
+       }()
+
+       select {
+       case <-statReadme:
+               break
+       case err := <-runner.ArvMountExit:
+               runner.ArvMount = nil
+               keepStatting = false
+               return nil, err
+       }
+
+       return c, nil
+}
+
 func (runner *ContainerRunner) SetupMounts() (err error) {
-       runner.ArvMountPoint, err = ioutil.TempDir("", "keep")
+       runner.ArvMountPoint, err = runner.MkTempDir("", "keep")
        if err != nil {
                return fmt.Errorf("While creating keep mount temp dir: %v", err)
        }
 
+       runner.CleanupTempDir = append(runner.CleanupTempDir, runner.ArvMountPoint)
+
        pdhOnly := true
        tmpcount := 0
-       arvMountCmd := []string{"--foreground"}
+       arvMountCmd := []string{"--foreground", "--allow-other", "--read-write"}
        collectionPaths := []string{}
+       runner.Binds = nil
 
        for bind, mnt := range runner.ContainerRecord.Mounts {
                if mnt.Kind == "collection" {
@@ -209,9 +263,9 @@ func (runner *ContainerRunner) SetupMounts() (err error) {
                                }
                                src = fmt.Sprintf("%s/by_id/%s", runner.ArvMountPoint, mnt.PortableDataHash)
                        } else {
-                               src = fmt.Sprintf("%s/tmp%i", runner.ArvMountPoint, tmpcount)
+                               src = fmt.Sprintf("%s/tmp%d", runner.ArvMountPoint, tmpcount)
                                arvMountCmd = append(arvMountCmd, "--mount-tmp")
-                               arvMountCmd = append(arvMountCmd, fmt.Sprintf("tmp%i", tmpcount))
+                               arvMountCmd = append(arvMountCmd, fmt.Sprintf("tmp%d", tmpcount))
                                tmpcount += 1
                        }
                        if mnt.Writable {
@@ -225,11 +279,19 @@ func (runner *ContainerRunner) SetupMounts() (err error) {
                        collectionPaths = append(collectionPaths, src)
                } else if mnt.Kind == "tmp" {
                        if bind == runner.ContainerRecord.OutputPath {
-                               runner.HostOutputDir, err = ioutil.TempDir("", "")
+                               runner.HostOutputDir, err = runner.MkTempDir("", "")
                                if err != nil {
                                        return fmt.Errorf("While creating mount temp dir: %v", err)
                                }
-
+                               st, staterr := os.Stat(runner.HostOutputDir)
+                               if staterr != nil {
+                                       return fmt.Errorf("While Stat on temp dir: %v", staterr)
+                               }
+                               err = os.Chmod(runner.HostOutputDir, st.Mode()|os.ModeSetgid|0777)
+                               if staterr != nil {
+                                       return fmt.Errorf("While Chmod temp dir: %v", err)
+                               }
+                               runner.CleanupTempDir = append(runner.CleanupTempDir, runner.HostOutputDir)
                                runner.Binds = append(runner.Binds, fmt.Sprintf("%s:%s", runner.HostOutputDir, bind))
                        } else {
                                runner.Binds = append(runner.Binds, bind)
@@ -250,10 +312,8 @@ func (runner *ContainerRunner) SetupMounts() (err error) {
        }
        arvMountCmd = append(arvMountCmd, runner.ArvMountPoint)
 
-       runner.ArvMount = exec.Command("arv-mount", arvMountCmd...)
-       err = runner.ArvMount.Start()
+       runner.ArvMount, err = runner.RunArvMount(arvMountCmd)
        if err != nil {
-               runner.ArvMount = nil
                return fmt.Errorf("While trying to start arv-mount: %v", err)
        }
 
@@ -267,6 +327,70 @@ func (runner *ContainerRunner) SetupMounts() (err error) {
        return nil
 }
 
+func (runner *ContainerRunner) ProcessDockerAttach(containerReader io.Reader) {
+       // Handle docker log protocol
+       // https://docs.docker.com/engine/reference/api/docker_remote_api_v1.15/#attach-to-a-container
+
+       header := make([]byte, 8)
+       for {
+               _, readerr := io.ReadAtLeast(containerReader, header, 8)
+
+               if readerr == nil {
+                       readsize := int64(header[7]) | (int64(header[6]) << 8) | (int64(header[5]) << 16) | (int64(header[4]) << 24)
+                       if header[0] == 1 {
+                               // stdout
+                               _, readerr = io.CopyN(runner.Stdout, containerReader, readsize)
+                       } else {
+                               // stderr
+                               _, readerr = io.CopyN(runner.Stderr, containerReader, readsize)
+                       }
+               }
+
+               if readerr != nil {
+                       if readerr != io.EOF {
+                               runner.CrunchLog.Printf("While reading docker logs: %v", readerr)
+                       }
+
+                       closeerr := runner.Stdout.Close()
+                       if closeerr != nil {
+                               runner.CrunchLog.Printf("While closing stdout logs: %v", closeerr)
+                       }
+
+                       closeerr = runner.Stderr.Close()
+                       if closeerr != nil {
+                               runner.CrunchLog.Printf("While closing stderr logs: %v", closeerr)
+                       }
+
+                       runner.loggingDone <- true
+                       close(runner.loggingDone)
+                       return
+               }
+       }
+}
+
+// AttachLogs connects the docker container stdout and stderr logs to the
+// Arvados logger which logs to Keep and the API server logs table.
+func (runner *ContainerRunner) AttachStreams() (err error) {
+
+       runner.CrunchLog.Print("Attaching container streams")
+
+       var containerReader io.Reader
+       containerReader, err = runner.Docker.AttachContainer(runner.ContainerID,
+               &dockerclient.AttachOptions{Stream: true, Stdout: true, Stderr: true})
+       if err != nil {
+               return fmt.Errorf("While attaching container stdout/stderr streams: %v", err)
+       }
+
+       runner.loggingDone = make(chan bool)
+
+       runner.Stdout = NewThrottledLogger(runner.NewLogWriter("stdout"))
+       runner.Stderr = NewThrottledLogger(runner.NewLogWriter("stderr"))
+
+       go runner.ProcessDockerAttach(containerReader)
+
+       return nil
+}
+
 // StartContainer creates the container and runs it.
 func (runner *ContainerRunner) StartContainer() (err error) {
        runner.CrunchLog.Print("Creating Docker container")
@@ -290,46 +414,28 @@ func (runner *ContainerRunner) StartContainer() (err error) {
        if err != nil {
                return fmt.Errorf("While creating container: %v", err)
        }
-       hostConfig := &dockerclient.HostConfig{Binds: runner.Binds}
+       hostConfig := &dockerclient.HostConfig{Binds: runner.Binds,
+               LogConfig: dockerclient.LogConfig{Type: "none"}}
 
-       runner.CrunchLog.Printf("Starting Docker container id '%s'", runner.ContainerID)
-       err = runner.Docker.StartContainer(runner.ContainerID, hostConfig)
+       err = runner.AttachStreams()
        if err != nil {
-               return fmt.Errorf("While starting container: %v", err)
+               return err
        }
 
-       return nil
-}
-
-// AttachLogs connects the docker container stdout and stderr logs to the
-// Arvados logger which logs to Keep and the API server logs table.
-func (runner *ContainerRunner) AttachLogs() (err error) {
-
-       runner.CrunchLog.Print("Attaching container logs")
-
-       var stderrReader, stdoutReader io.Reader
-       stderrReader, err = runner.Docker.ContainerLogs(runner.ContainerID, &dockerclient.LogOptions{Follow: true, Stderr: true})
-       if err != nil {
-               return fmt.Errorf("While getting container standard error: %v", err)
-       }
-       stdoutReader, err = runner.Docker.ContainerLogs(runner.ContainerID, &dockerclient.LogOptions{Follow: true, Stdout: true})
+       runner.CrunchLog.Printf("Starting Docker container id '%s'", runner.ContainerID)
+       err = runner.Docker.StartContainer(runner.ContainerID, hostConfig)
        if err != nil {
-               return fmt.Errorf("While getting container standard output: %v", err)
+               return fmt.Errorf("While starting container: %v", err)
        }
 
-       runner.loggingDone = make(chan bool)
-
-       runner.Stdout = NewThrottledLogger(runner.NewLogWriter("stdout"))
-       runner.Stderr = NewThrottledLogger(runner.NewLogWriter("stderr"))
-       go ReadWriteLines(stdoutReader, runner.Stdout, runner.loggingDone)
-       go ReadWriteLines(stderrReader, runner.Stderr, runner.loggingDone)
-
        return nil
 }
 
 // WaitFinish waits for the container to terminate, capture the exit code, and
 // close the stdout/stderr logging.
 func (runner *ContainerRunner) WaitFinish() error {
+       runner.CrunchLog.Print("Waiting for container to finish")
+
        result := runner.Docker.Wait(runner.ContainerID)
        wr := <-result
        if wr.Error != nil {
@@ -337,25 +443,14 @@ func (runner *ContainerRunner) WaitFinish() error {
        }
        runner.ExitCode = &wr.ExitCode
 
-       // drain stdout/stderr
+       // wait for stdout/stderr to complete
        <-runner.loggingDone
-       <-runner.loggingDone
-
-       runner.Stdout.Close()
-       runner.Stderr.Close()
 
        return nil
 }
 
-// HandleOutput sets the output and unmounts the FUSE mount.
+// HandleOutput sets the output, unmounts the FUSE mount, and deletes temporary directories
 func (runner *ContainerRunner) CaptureOutput() error {
-       if runner.ArvMount != nil {
-               defer func() {
-                       umount := exec.Command("fusermount", "-z", "-u", runner.ArvMountPoint)
-                       umount.Run()
-               }()
-       }
-
        if runner.finalState != "Complete" {
                return nil
        }
@@ -398,7 +493,9 @@ func (runner *ContainerRunner) CaptureOutput() error {
 
        var response CollectionRecord
        err = runner.ArvClient.Create("collections",
-               arvadosclient.Dict{"manifest_text": manifestText},
+               arvadosclient.Dict{
+                       "collection": arvadosclient.Dict{
+                               "manifest_text": manifestText}},
                &response)
        if err != nil {
                return fmt.Errorf("While creating output collection: %v", err)
@@ -410,6 +507,28 @@ func (runner *ContainerRunner) CaptureOutput() error {
        return nil
 }
 
+func (runner *ContainerRunner) CleanupDirs() {
+       if runner.ArvMount != nil {
+               umount := exec.Command("fusermount", "-z", "-u", runner.ArvMountPoint)
+               umnterr := umount.Run()
+               if umnterr != nil {
+                       runner.CrunchLog.Printf("While running fusermount: %v", umnterr)
+               }
+
+               mnterr := <-runner.ArvMountExit
+               if mnterr != nil {
+                       runner.CrunchLog.Printf("Arv-mount exit error: %v", mnterr)
+               }
+       }
+
+       for _, tmpdir := range runner.CleanupTempDir {
+               rmerr := os.RemoveAll(tmpdir)
+               if rmerr != nil {
+                       runner.CrunchLog.Printf("While cleaning up temporary directory %s: %v", tmpdir, rmerr)
+               }
+       }
+}
+
 // CommitLogs posts the collection containing the final container logs.
 func (runner *ContainerRunner) CommitLogs() error {
        runner.CrunchLog.Print(runner.finalState)
@@ -429,8 +548,10 @@ func (runner *ContainerRunner) CommitLogs() error {
 
        var response CollectionRecord
        err = runner.ArvClient.Create("collections",
-               arvadosclient.Dict{"name": "logs for " + runner.ContainerRecord.UUID,
-                       "manifest_text": mt},
+               arvadosclient.Dict{
+                       "collection": arvadosclient.Dict{
+                               "name":          "logs for " + runner.ContainerRecord.UUID,
+                               "manifest_text": mt}},
                &response)
        if err != nil {
                return fmt.Errorf("While creating log collection: %v", err)
@@ -444,8 +565,8 @@ func (runner *ContainerRunner) CommitLogs() error {
 
 // UpdateContainerRecordRunning updates the container state to "Running"
 func (runner *ContainerRunner) UpdateContainerRecordRunning() error {
-       update := arvadosclient.Dict{"state": "Running"}
-       return runner.ArvClient.Update("containers", runner.ContainerRecord.UUID, update, nil)
+       return runner.ArvClient.Update("containers", runner.ContainerRecord.UUID,
+               arvadosclient.Dict{"container": arvadosclient.Dict{"state": "Running"}}, nil)
 }
 
 // UpdateContainerRecordComplete updates the container record state on API
@@ -464,7 +585,7 @@ func (runner *ContainerRunner) UpdateContainerRecordComplete() error {
 
        update["state"] = runner.finalState
 
-       return runner.ArvClient.Update("containers", runner.ContainerRecord.UUID, update, nil)
+       return runner.ArvClient.Update("containers", runner.ContainerRecord.UUID, arvadosclient.Dict{"container": update}, nil)
 }
 
 // NewArvLogWriter creates an ArvLogWriter
@@ -473,8 +594,15 @@ func (runner *ContainerRunner) NewArvLogWriter(name string) io.WriteCloser {
 }
 
 // Run the full container lifecycle.
-func (runner *ContainerRunner) Run(containerUUID string) (err error) {
-       runner.CrunchLog.Printf("Executing container '%s'", containerUUID)
+func (runner *ContainerRunner) Run() (err error) {
+       runner.CrunchLog.Printf("Executing container '%s'", runner.ContainerRecord.UUID)
+
+       hostname, hosterr := os.Hostname()
+       if hosterr != nil {
+               runner.CrunchLog.Printf("Error getting hostname '%v'", hosterr)
+       } else {
+               runner.CrunchLog.Printf("Executing on host '%s'", hostname)
+       }
 
        var runerr, waiterr error
 
@@ -489,12 +617,15 @@ func (runner *ContainerRunner) Run(containerUUID string) (err error) {
                        runner.finalState = "Complete"
                }
 
-               // (7) capture output
+               // (6) capture output
                outputerr := runner.CaptureOutput()
                if outputerr != nil {
                        runner.CrunchLog.Print(outputerr)
                }
 
+               // (7) clean up temporary directories
+               runner.CleanupDirs()
+
                // (8) write logs
                logerr := runner.CommitLogs()
                if logerr != nil {
@@ -513,7 +644,7 @@ func (runner *ContainerRunner) Run(containerUUID string) (err error) {
                        if runerr != nil {
                                err = runerr
                        } else if waiterr != nil {
-                               err = runerr
+                               err = waiterr
                        } else if logerr != nil {
                                err = logerr
                        } else if updateerr != nil {
@@ -522,16 +653,13 @@ func (runner *ContainerRunner) Run(containerUUID string) (err error) {
                }
        }()
 
-       err = runner.ArvClient.Get("containers", containerUUID, nil, &runner.ContainerRecord)
+       err = runner.ArvClient.Get("containers", runner.ContainerRecord.UUID, nil, &runner.ContainerRecord)
        if err != nil {
                return fmt.Errorf("While getting container record: %v", err)
        }
 
        // (1) setup signal handling
-       err = runner.SetupSignals()
-       if err != nil {
-               return fmt.Errorf("While setting up signal handling: %v", err)
-       }
+       runner.SetupSignals()
 
        // (2) check for and/or load image
        err = runner.LoadImage()
@@ -560,13 +688,7 @@ func (runner *ContainerRunner) Run(containerUUID string) (err error) {
                runner.CrunchLog.Print(err)
        }
 
-       // (5) attach container logs
-       runerr = runner.AttachLogs()
-       if runerr != nil {
-               runner.CrunchLog.Print(runerr)
-       }
-
-       // (6) wait for container to finish
+       // (5) wait for container to finish
        waiterr = runner.WaitFinish()
 
        return
@@ -575,42 +697,49 @@ func (runner *ContainerRunner) Run(containerUUID string) (err error) {
 // NewContainerRunner creates a new container runner.
 func NewContainerRunner(api IArvadosClient,
        kc IKeepClient,
-       docker ThinDockerClient) *ContainerRunner {
+       docker ThinDockerClient,
+       containerUUID string) *ContainerRunner {
 
        cr := &ContainerRunner{ArvClient: api, Kc: kc, Docker: docker}
        cr.NewLogWriter = cr.NewArvLogWriter
+       cr.RunArvMount = cr.ArvMountCmd
+       cr.MkTempDir = ioutil.TempDir
        cr.LogCollection = &CollectionWriter{kc, nil, sync.Mutex{}}
+       cr.ContainerRecord.UUID = containerUUID
        cr.CrunchLog = NewThrottledLogger(cr.NewLogWriter("crunch-run"))
+       cr.CrunchLog.Immediate = log.New(os.Stderr, containerUUID+" ", 0)
        return cr
 }
 
 func main() {
        flag.Parse()
 
+       containerId := flag.Arg(0)
+
        api, err := arvadosclient.MakeArvadosClient()
        if err != nil {
-               log.Fatal(err)
+               log.Fatalf("%s: %v", containerId, err)
        }
        api.Retries = 8
 
        var kc *keepclient.KeepClient
        kc, err = keepclient.MakeKeepClient(&api)
        if err != nil {
-               log.Fatal(err)
+               log.Fatalf("%s: %v", containerId, err)
        }
        kc.Retries = 4
 
        var docker *dockerclient.DockerClient
        docker, err = dockerclient.NewDockerClient("unix:///var/run/docker.sock", nil)
        if err != nil {
-               log.Fatal(err)
+               log.Fatalf("%s: %v", containerId, err)
        }
 
-       cr := NewContainerRunner(api, kc, docker)
+       cr := NewContainerRunner(api, kc, docker, containerId)
 
-       err = cr.Run(flag.Arg(0))
+       err = cr.Run()
        if err != nil {
-               log.Fatal(err)
+               log.Fatalf("%s: %v", containerId, err)
        }
 
 }