Browse Source

Add provenance pull flow for official images

Add support for pulling signed images from a version 2 registry.
Only official images within the library namespace will be pull from the
new registry and check the build signature.

Signed-off-by: Derek McGowan <derek@mcgstyle.net> (github: dmcgowan)
Derek McGowan 10 years ago
parent
commit
7c88e8f13d
11 changed files with 971 additions and 13 deletions
  1. 15 0
      daemon/daemon.go
  2. 240 1
      graph/pull.go
  3. 14 0
      graph/tags.go
  4. 20 4
      graph/tags_unit_test.go
  5. 1 0
      registry/registry.go
  6. 6 0
      registry/registry_mock_test.go
  7. 6 6
      registry/session.go
  8. 386 0
      registry/session_v2.go
  9. 10 2
      registry/types.go
  10. 74 0
      trust/service.go
  11. 199 0
      trust/trusts.go

+ 15 - 0
daemon/daemon.go

@@ -38,6 +38,7 @@ import (
 	"github.com/docker/docker/pkg/sysinfo"
 	"github.com/docker/docker/pkg/truncindex"
 	"github.com/docker/docker/runconfig"
+	"github.com/docker/docker/trust"
 	"github.com/docker/docker/utils"
 	"github.com/docker/docker/volumes"
 )
@@ -98,6 +99,7 @@ type Daemon struct {
 	containerGraph *graphdb.Database
 	driver         graphdriver.Driver
 	execDriver     execdriver.Driver
+	trustStore     *trust.TrustStore
 }
 
 // Install installs daemon capabilities to eng.
@@ -136,6 +138,9 @@ func (daemon *Daemon) Install(eng *engine.Engine) error {
 	if err := daemon.Repositories().Install(eng); err != nil {
 		return err
 	}
+	if err := daemon.trustStore.Install(eng); err != nil {
+		return err
+	}
 	// FIXME: this hack is necessary for legacy integration tests to access
 	// the daemon object.
 	eng.Hack_SetGlobalVar("httpapi.daemon", daemon)
@@ -813,6 +818,15 @@ func NewDaemonFromDirectory(config *Config, eng *engine.Engine) (*Daemon, error)
 		return nil, fmt.Errorf("Couldn't create Tag store: %s", err)
 	}
 
+	trustDir := path.Join(config.Root, "trust")
+	if err := os.MkdirAll(trustDir, 0700); err != nil && !os.IsExist(err) {
+		return nil, err
+	}
+	t, err := trust.NewTrustStore(trustDir)
+	if err != nil {
+		return nil, fmt.Errorf("could not create trust store: %s", err)
+	}
+
 	if !config.DisableNetwork {
 		job := eng.Job("init_networkdriver")
 
@@ -877,6 +891,7 @@ func NewDaemonFromDirectory(config *Config, eng *engine.Engine) (*Daemon, error)
 		sysInitPath:    sysInitPath,
 		execDriver:     ed,
 		eng:            eng,
+		trustStore:     t,
 	}
 	if err := daemon.checkLocaldns(); err != nil {
 		return nil, err

+ 240 - 1
graph/pull.go

@@ -1,10 +1,14 @@
 package graph
 
 import (
+	"bytes"
+	"encoding/json"
 	"fmt"
 	"io"
+	"io/ioutil"
 	"net"
 	"net/url"
+	"os"
 	"strings"
 	"time"
 
@@ -13,8 +17,59 @@ import (
 	"github.com/docker/docker/pkg/log"
 	"github.com/docker/docker/registry"
 	"github.com/docker/docker/utils"
+	"github.com/docker/libtrust"
 )
 
+func (s *TagStore) verifyManifest(eng *engine.Engine, manifestBytes []byte) (*registry.ManifestData, bool, error) {
+	sig, err := libtrust.ParsePrettySignature(manifestBytes, "signatures")
+	if err != nil {
+		return nil, false, fmt.Errorf("error parsing payload: %s", err)
+	}
+	keys, err := sig.Verify()
+	if err != nil {
+		return nil, false, fmt.Errorf("error verifying payload: %s", err)
+	}
+
+	payload, err := sig.Payload()
+	if err != nil {
+		return nil, false, fmt.Errorf("error retrieving payload: %s", err)
+	}
+
+	var manifest registry.ManifestData
+	if err := json.Unmarshal(payload, &manifest); err != nil {
+		return nil, false, fmt.Errorf("error unmarshalling manifest: %s", err)
+	}
+
+	var verified bool
+	for _, key := range keys {
+		job := eng.Job("trust_key_check")
+		b, err := key.MarshalJSON()
+		if err != nil {
+			return nil, false, fmt.Errorf("error marshalling public key: %s", err)
+		}
+		namespace := manifest.Name
+		if namespace[0] != '/' {
+			namespace = "/" + namespace
+		}
+		stdoutBuffer := bytes.NewBuffer(nil)
+
+		job.Args = append(job.Args, namespace)
+		job.Setenv("PublicKey", string(b))
+		job.SetenvInt("Permission", 0x03)
+		job.Stdout.Add(stdoutBuffer)
+		if err = job.Run(); err != nil {
+			return nil, false, fmt.Errorf("error running key check: %s", err)
+		}
+		result := engine.Tail(stdoutBuffer, 1)
+		log.Debugf("Key check result: %q", result)
+		if result == "verified" {
+			verified = true
+		}
+	}
+
+	return &manifest, verified, nil
+}
+
 func (s *TagStore) CmdPull(job *engine.Job) engine.Status {
 	if n := len(job.Args); n != 1 && n != 2 {
 		return job.Errorf("Usage: %s IMAGE [TAG]", job.Name)
@@ -62,14 +117,32 @@ func (s *TagStore) CmdPull(job *engine.Job) engine.Status {
 		return job.Error(err)
 	}
 
-	if endpoint.String() == registry.IndexServerAddress() {
+	var isOfficial bool
+	if endpoint.VersionString(1) == registry.IndexServerAddress() {
 		// If pull "index.docker.io/foo/bar", it's stored locally under "foo/bar"
 		localName = remoteName
 
+		isOfficial = isOfficialName(remoteName)
+		if isOfficial && strings.IndexRune(remoteName, '/') == -1 {
+			remoteName = "library/" + remoteName
+		}
+
 		// Use provided mirrors, if any
 		mirrors = s.mirrors
 	}
 
+	if isOfficial || endpoint.Version == registry.APIVersion2 {
+		j := job.Eng.Job("trust_update_base")
+		if err = j.Run(); err != nil {
+			return job.Errorf("error updating trust base graph: %s", err)
+		}
+
+		if err := s.pullV2Repository(job.Eng, r, job.Stdout, localName, remoteName, tag, sf, job.GetenvBool("parallel")); err == nil {
+			return engine.StatusOK
+		} else if err != registry.ErrDoesNotExist {
+			log.Errorf("Error from V2 registry: %s", err)
+		}
+	}
 	if err = s.pullRepository(r, job.Stdout, localName, remoteName, tag, sf, job.GetenvBool("parallel"), mirrors); err != nil {
 		return job.Error(err)
 	}
@@ -317,3 +390,169 @@ func (s *TagStore) pullImage(r *registry.Session, out io.Writer, imgID, endpoint
 	}
 	return nil
 }
+
+// downloadInfo is used to pass information from download to extractor
+type downloadInfo struct {
+	imgJSON    []byte
+	img        *image.Image
+	tmpFile    *os.File
+	length     int64
+	downloaded bool
+	err        chan error
+}
+
+func (s *TagStore) pullV2Repository(eng *engine.Engine, r *registry.Session, out io.Writer, localName, remoteName, tag string, sf *utils.StreamFormatter, parallel bool) error {
+	if tag == "" {
+		log.Debugf("Pulling tag list from V2 registry for %s", remoteName)
+		tags, err := r.GetV2RemoteTags(remoteName, nil)
+		if err != nil {
+			return err
+		}
+		for _, t := range tags {
+			if err := s.pullV2Tag(eng, r, out, localName, remoteName, t, sf, parallel); err != nil {
+				return err
+			}
+		}
+	} else {
+		if err := s.pullV2Tag(eng, r, out, localName, remoteName, tag, sf, parallel); err != nil {
+			return err
+		}
+	}
+
+	return nil
+}
+
+func (s *TagStore) pullV2Tag(eng *engine.Engine, r *registry.Session, out io.Writer, localName, remoteName, tag string, sf *utils.StreamFormatter, parallel bool) error {
+	log.Debugf("Pulling tag from V2 registry: %q", tag)
+	manifestBytes, err := r.GetV2ImageManifest(remoteName, tag, nil)
+	if err != nil {
+		return err
+	}
+
+	manifest, verified, err := s.verifyManifest(eng, manifestBytes)
+	if err != nil {
+		return fmt.Errorf("error verifying manifest: %s", err)
+	}
+
+	if len(manifest.BlobSums) != len(manifest.History) {
+		return fmt.Errorf("length of history not equal to number of layers")
+	}
+
+	if verified {
+		out.Write(sf.FormatStatus("", "The image you are pulling has been digitally signed by Docker, Inc."))
+	}
+	out.Write(sf.FormatStatus(tag, "Pulling from %s", localName))
+
+	downloads := make([]downloadInfo, len(manifest.BlobSums))
+
+	for i := len(manifest.BlobSums) - 1; i >= 0; i-- {
+		var (
+			sumStr  = manifest.BlobSums[i]
+			imgJSON = []byte(manifest.History[i])
+		)
+
+		img, err := image.NewImgJSON(imgJSON)
+		if err != nil {
+			return fmt.Errorf("failed to parse json: %s", err)
+		}
+		downloads[i].img = img
+
+		// Check if exists
+		if s.graph.Exists(img.ID) {
+			log.Debugf("Image already exists: %s", img.ID)
+			continue
+		}
+
+		chunks := strings.SplitN(sumStr, ":", 2)
+		if len(chunks) < 2 {
+			return fmt.Errorf("expected 2 parts in the sumStr, got %#v", chunks)
+		}
+		sumType, checksum := chunks[0], chunks[1]
+		out.Write(sf.FormatProgress(utils.TruncateID(img.ID), "Pulling fs layer", nil))
+
+		downloadFunc := func(di *downloadInfo) error {
+			log.Infof("pulling blob %q to V1 img %s", sumStr, img.ID)
+
+			if c, err := s.poolAdd("pull", "img:"+img.ID); err != nil {
+				if c != nil {
+					out.Write(sf.FormatProgress(utils.TruncateID(img.ID), "Layer already being pulled by another client. Waiting.", nil))
+					<-c
+					out.Write(sf.FormatProgress(utils.TruncateID(img.ID), "Download complete", nil))
+				} else {
+					log.Debugf("Image (id: %s) pull is already running, skipping: %v", img.ID, err)
+				}
+			} else {
+				tmpFile, err := ioutil.TempFile("", "GetV2ImageBlob")
+				if err != nil {
+					return err
+				}
+
+				r, l, err := r.GetV2ImageBlobReader(remoteName, sumType, checksum, nil)
+				if err != nil {
+					return err
+				}
+				defer r.Close()
+				io.Copy(tmpFile, utils.ProgressReader(r, int(l), out, sf, false, utils.TruncateID(img.ID), "Downloading"))
+
+				out.Write(sf.FormatProgress(utils.TruncateID(img.ID), "Download complete", nil))
+
+				log.Debugf("Downloaded %s to tempfile %s", img.ID, tmpFile.Name())
+				di.tmpFile = tmpFile
+				di.length = l
+				di.downloaded = true
+			}
+			di.imgJSON = imgJSON
+			defer s.poolRemove("pull", "img:"+img.ID)
+
+			return nil
+		}
+
+		if parallel {
+			downloads[i].err = make(chan error)
+			go func(di *downloadInfo) {
+				di.err <- downloadFunc(di)
+			}(&downloads[i])
+		} else {
+			err := downloadFunc(&downloads[i])
+			if err != nil {
+				return err
+			}
+		}
+	}
+
+	for i := len(downloads) - 1; i >= 0; i-- {
+		d := &downloads[i]
+		if d.err != nil {
+			err := <-d.err
+			if err != nil {
+				return err
+			}
+		}
+		if d.downloaded {
+			// if tmpFile is empty assume download and extracted elsewhere
+			defer os.Remove(d.tmpFile.Name())
+			defer d.tmpFile.Close()
+			d.tmpFile.Seek(0, 0)
+			if d.tmpFile != nil {
+				err = s.graph.Register(d.img, d.imgJSON,
+					utils.ProgressReader(d.tmpFile, int(d.length), out, sf, false, utils.TruncateID(d.img.ID), "Extracting"))
+				if err != nil {
+					return err
+				}
+
+				// FIXME: Pool release here for parallel tag pull (ensures any downloads block until fully extracted)
+			}
+			out.Write(sf.FormatProgress(utils.TruncateID(d.img.ID), "Pull complete", nil))
+
+		} else {
+			out.Write(sf.FormatProgress(utils.TruncateID(d.img.ID), "Already exists", nil))
+		}
+
+	}
+
+	if err = s.Set(localName, tag, downloads[0].img.ID, true); err != nil {
+		return err
+	}
+
+	return nil
+}

+ 14 - 0
graph/tags.go

@@ -276,6 +276,20 @@ func (store *TagStore) GetRepoRefs() map[string][]string {
 	return reporefs
 }
 
+// isOfficialName returns whether a repo name is considered an official
+// repository.  Official repositories are repos with names within
+// the library namespace or which default to the library namespace
+// by not providing one.
+func isOfficialName(name string) bool {
+	if strings.HasPrefix(name, "library/") {
+		return true
+	}
+	if strings.IndexRune(name, '/') == -1 {
+		return true
+	}
+	return false
+}
+
 // Validate the name of a repository
 func validateRepoName(name string) error {
 	if name == "" {

+ 20 - 4
graph/tags_unit_test.go

@@ -2,15 +2,16 @@ package graph
 
 import (
 	"bytes"
+	"io"
+	"os"
+	"path"
+	"testing"
+
 	"github.com/docker/docker/daemon/graphdriver"
 	_ "github.com/docker/docker/daemon/graphdriver/vfs" // import the vfs driver so it is used in the tests
 	"github.com/docker/docker/image"
 	"github.com/docker/docker/utils"
 	"github.com/docker/docker/vendor/src/code.google.com/p/go/src/pkg/archive/tar"
-	"io"
-	"os"
-	"path"
-	"testing"
 )
 
 const (
@@ -132,3 +133,18 @@ func TestInvalidTagName(t *testing.T) {
 		}
 	}
 }
+
+func TestOfficialName(t *testing.T) {
+	names := map[string]bool{
+		"library/ubuntu":    true,
+		"nonlibrary/ubuntu": false,
+		"ubuntu":            true,
+		"other/library":     false,
+	}
+	for name, isOfficial := range names {
+		result := isOfficialName(name)
+		if result != isOfficial {
+			t.Errorf("Unexpected result for %s\n\tExpecting: %v\n\tActual: %v", name, isOfficial, result)
+		}
+	}
+}

+ 1 - 0
registry/registry.go

@@ -20,6 +20,7 @@ import (
 var (
 	ErrAlreadyExists         = errors.New("Image already exists")
 	ErrInvalidRepositoryName = errors.New("Invalid repository name (ex: \"registry.domain.tld/myrepos\")")
+	ErrDoesNotExist          = errors.New("Image does not exist")
 	errLoginRequired         = errors.New("Authentication is required.")
 	validHex                 = regexp.MustCompile(`^([a-f0-9]{64})$`)
 	validNamespace           = regexp.MustCompile(`^([a-z0-9_]{4,30})$`)

+ 6 - 0
registry/registry_mock_test.go

@@ -83,6 +83,8 @@ var (
 
 func init() {
 	r := mux.NewRouter()
+
+	// /v1/
 	r.HandleFunc("/v1/_ping", handlerGetPing).Methods("GET")
 	r.HandleFunc("/v1/images/{image_id:[^/]+}/{action:json|layer|ancestry}", handlerGetImage).Methods("GET")
 	r.HandleFunc("/v1/images/{image_id:[^/]+}/{action:json|layer|checksum}", handlerPutImage).Methods("PUT")
@@ -93,6 +95,10 @@ func init() {
 	r.HandleFunc("/v1/repositories/{repository:.+}{action:/images|/}", handlerImages).Methods("GET", "PUT", "DELETE")
 	r.HandleFunc("/v1/repositories/{repository:.+}/auth", handlerAuth).Methods("PUT")
 	r.HandleFunc("/v1/search", handlerSearch).Methods("GET")
+
+	// /v2/
+	r.HandleFunc("/v2/version", handlerGetPing).Methods("GET")
+
 	testHttpServer = httptest.NewServer(handlerAccessLog(r))
 }
 

+ 6 - 6
registry/session.go

@@ -47,7 +47,7 @@ func NewSession(authConfig *AuthConfig, factory *utils.HTTPRequestFactory, endpo
 
 	// If we're working with a standalone private registry over HTTPS, send Basic Auth headers
 	// alongside our requests.
-	if r.indexEndpoint.String() != IndexServerAddress() && r.indexEndpoint.URL.Scheme == "https" {
+	if r.indexEndpoint.VersionString(1) != IndexServerAddress() && r.indexEndpoint.URL.Scheme == "https" {
 		info, err := r.indexEndpoint.Ping()
 		if err != nil {
 			return nil, err
@@ -261,7 +261,7 @@ func buildEndpointsList(headers []string, indexEp string) ([]string, error) {
 }
 
 func (r *Session) GetRepositoryData(remote string) (*RepositoryData, error) {
-	repositoryTarget := fmt.Sprintf("%srepositories/%s/images", r.indexEndpoint.String(), remote)
+	repositoryTarget := fmt.Sprintf("%srepositories/%s/images", r.indexEndpoint.VersionString(1), remote)
 
 	log.Debugf("[registry] Calling GET %s", repositoryTarget)
 
@@ -295,7 +295,7 @@ func (r *Session) GetRepositoryData(remote string) (*RepositoryData, error) {
 
 	var endpoints []string
 	if res.Header.Get("X-Docker-Endpoints") != "" {
-		endpoints, err = buildEndpointsList(res.Header["X-Docker-Endpoints"], r.indexEndpoint.String())
+		endpoints, err = buildEndpointsList(res.Header["X-Docker-Endpoints"], r.indexEndpoint.VersionString(1))
 		if err != nil {
 			return nil, err
 		}
@@ -488,7 +488,7 @@ func (r *Session) PushImageJSONIndex(remote string, imgList []*ImgData, validate
 	if validate {
 		suffix = "images"
 	}
-	u := fmt.Sprintf("%srepositories/%s/%s", r.indexEndpoint.String(), remote, suffix)
+	u := fmt.Sprintf("%srepositories/%s/%s", r.indexEndpoint.VersionString(1), remote, suffix)
 	log.Debugf("[registry] PUT %s", u)
 	log.Debugf("Image list pushed to index:\n%s", imgListJSON)
 	req, err := r.reqFactory.NewRequest("PUT", u, bytes.NewReader(imgListJSON))
@@ -546,7 +546,7 @@ func (r *Session) PushImageJSONIndex(remote string, imgList []*ImgData, validate
 		}
 
 		if res.Header.Get("X-Docker-Endpoints") != "" {
-			endpoints, err = buildEndpointsList(res.Header["X-Docker-Endpoints"], r.indexEndpoint.String())
+			endpoints, err = buildEndpointsList(res.Header["X-Docker-Endpoints"], r.indexEndpoint.VersionString(1))
 			if err != nil {
 				return nil, err
 			}
@@ -572,7 +572,7 @@ func (r *Session) PushImageJSONIndex(remote string, imgList []*ImgData, validate
 
 func (r *Session) SearchRepositories(term string) (*SearchResults, error) {
 	log.Debugf("Index server: %s", r.indexEndpoint)
-	u := r.indexEndpoint.String() + "search?q=" + url.QueryEscape(term)
+	u := r.indexEndpoint.VersionString(1) + "search?q=" + url.QueryEscape(term)
 	req, err := r.reqFactory.NewRequest("GET", u, nil)
 	if err != nil {
 		return nil, err

+ 386 - 0
registry/session_v2.go

@@ -0,0 +1,386 @@
+package registry
+
+import (
+	"encoding/json"
+	"fmt"
+	"io"
+	"io/ioutil"
+	"net/url"
+	"strconv"
+
+	"github.com/docker/docker/pkg/log"
+	"github.com/docker/docker/utils"
+	"github.com/gorilla/mux"
+)
+
+func newV2RegistryRouter() *mux.Router {
+	router := mux.NewRouter()
+
+	v2Router := router.PathPrefix("/v2/").Subrouter()
+
+	// Version Info
+	v2Router.Path("/version").Name("version")
+
+	// Image Manifests
+	v2Router.Path("/manifest/{imagename:[a-z0-9-._/]+}/{tagname:[a-zA-Z0-9-._]+}").Name("manifests")
+
+	// List Image Tags
+	v2Router.Path("/tags/{imagename:[a-z0-9-._/]+}").Name("tags")
+
+	// Download a blob
+	v2Router.Path("/blob/{imagename:[a-z0-9-._/]+}/{sumtype:[a-z0-9_+-]+}/{sum:[a-fA-F0-9]{4,}}").Name("downloadBlob")
+
+	// Upload a blob
+	v2Router.Path("/blob/{imagename:[a-z0-9-._/]+}/{sumtype:[a-z0-9_+-]+}").Name("uploadBlob")
+
+	// Mounting a blob in an image
+	v2Router.Path("/mountblob/{imagename:[a-z0-9-._/]+}/{sumtype:[a-z0-9_+-]+}/{sum:[a-fA-F0-9]{4,}}").Name("mountBlob")
+
+	return router
+}
+
+// APIVersion2 /v2/
+var v2HTTPRoutes = newV2RegistryRouter()
+
+func getV2URL(e *Endpoint, routeName string, vars map[string]string) (*url.URL, error) {
+	route := v2HTTPRoutes.Get(routeName)
+	if route == nil {
+		return nil, fmt.Errorf("unknown regisry v2 route name: %q", routeName)
+	}
+
+	varReplace := make([]string, 0, len(vars)*2)
+	for key, val := range vars {
+		varReplace = append(varReplace, key, val)
+	}
+
+	routePath, err := route.URLPath(varReplace...)
+	if err != nil {
+		return nil, fmt.Errorf("unable to make registry route %q with vars %v: %s", routeName, vars, err)
+	}
+
+	return &url.URL{
+		Scheme: e.URL.Scheme,
+		Host:   e.URL.Host,
+		Path:   routePath.Path,
+	}, nil
+}
+
+// V2 Provenance POC
+
+func (r *Session) GetV2Version(token []string) (*RegistryInfo, error) {
+	routeURL, err := getV2URL(r.indexEndpoint, "version", nil)
+	if err != nil {
+		return nil, err
+	}
+
+	method := "GET"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), nil)
+	if err != nil {
+		return nil, err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return nil, err
+	}
+	defer res.Body.Close()
+	if res.StatusCode != 200 {
+		return nil, utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d fetching Version", res.StatusCode), res)
+	}
+
+	decoder := json.NewDecoder(res.Body)
+	versionInfo := new(RegistryInfo)
+
+	err = decoder.Decode(versionInfo)
+	if err != nil {
+		return nil, fmt.Errorf("unable to decode GetV2Version JSON response: %s", err)
+	}
+
+	return versionInfo, nil
+}
+
+//
+// 1) Check if TarSum of each layer exists /v2/
+//  1.a) if 200, continue
+//  1.b) if 300, then push the
+//  1.c) if anything else, err
+// 2) PUT the created/signed manifest
+//
+func (r *Session) GetV2ImageManifest(imageName, tagName string, token []string) ([]byte, error) {
+	vars := map[string]string{
+		"imagename": imageName,
+		"tagname":   tagName,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "manifests", vars)
+	if err != nil {
+		return nil, err
+	}
+
+	method := "GET"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), nil)
+	if err != nil {
+		return nil, err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return nil, err
+	}
+	defer res.Body.Close()
+	if res.StatusCode != 200 {
+		if res.StatusCode == 401 {
+			return nil, errLoginRequired
+		} else if res.StatusCode == 404 {
+			return nil, ErrDoesNotExist
+		}
+		return nil, utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d trying to fetch for %s:%s", res.StatusCode, imageName, tagName), res)
+	}
+
+	buf, err := ioutil.ReadAll(res.Body)
+	if err != nil {
+		return nil, fmt.Errorf("Error while reading the http response: %s", err)
+	}
+	return buf, nil
+}
+
+// - Succeeded to mount for this image scope
+// - Failed with no error (So continue to Push the Blob)
+// - Failed with error
+func (r *Session) PostV2ImageMountBlob(imageName, sumType, sum string, token []string) (bool, error) {
+	vars := map[string]string{
+		"imagename": imageName,
+		"sumtype":   sumType,
+		"sum":       sum,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "mountBlob", vars)
+	if err != nil {
+		return false, err
+	}
+
+	method := "POST"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), nil)
+	if err != nil {
+		return false, err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return false, err
+	}
+	res.Body.Close() // close early, since we're not needing a body on this call .. yet?
+	switch res.StatusCode {
+	case 200:
+		// return something indicating no push needed
+		return true, nil
+	case 300:
+		// return something indicating blob push needed
+		return false, nil
+	}
+	return false, fmt.Errorf("Failed to mount %q - %s:%s : %d", imageName, sumType, sum, res.StatusCode)
+}
+
+func (r *Session) GetV2ImageBlob(imageName, sumType, sum string, blobWrtr io.Writer, token []string) error {
+	vars := map[string]string{
+		"imagename": imageName,
+		"sumtype":   sumType,
+		"sum":       sum,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "downloadBlob", vars)
+	if err != nil {
+		return err
+	}
+
+	method := "GET"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), nil)
+	if err != nil {
+		return err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return err
+	}
+	defer res.Body.Close()
+	if res.StatusCode != 200 {
+		if res.StatusCode == 401 {
+			return errLoginRequired
+		}
+		return utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d trying to pull %s blob", res.StatusCode, imageName), res)
+	}
+
+	_, err = io.Copy(blobWrtr, res.Body)
+	return err
+}
+
+func (r *Session) GetV2ImageBlobReader(imageName, sumType, sum string, token []string) (io.ReadCloser, int64, error) {
+	vars := map[string]string{
+		"imagename": imageName,
+		"sumtype":   sumType,
+		"sum":       sum,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "downloadBlob", vars)
+	if err != nil {
+		return nil, 0, err
+	}
+
+	method := "GET"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), nil)
+	if err != nil {
+		return nil, 0, err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return nil, 0, err
+	}
+	if res.StatusCode != 200 {
+		if res.StatusCode == 401 {
+			return nil, 0, errLoginRequired
+		}
+		return nil, 0, utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d trying to pull %s blob", res.StatusCode, imageName), res)
+	}
+	lenStr := res.Header.Get("Content-Length")
+	l, err := strconv.ParseInt(lenStr, 10, 64)
+	if err != nil {
+		return nil, 0, err
+	}
+
+	return res.Body, l, err
+}
+
+// Push the image to the server for storage.
+// 'layer' is an uncompressed reader of the blob to be pushed.
+// The server will generate it's own checksum calculation.
+func (r *Session) PutV2ImageBlob(imageName, sumType string, blobRdr io.Reader, token []string) (serverChecksum string, err error) {
+	vars := map[string]string{
+		"imagename": imageName,
+		"sumtype":   sumType,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "uploadBlob", vars)
+	if err != nil {
+		return "", err
+	}
+
+	method := "PUT"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), blobRdr)
+	if err != nil {
+		return "", err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return "", err
+	}
+	defer res.Body.Close()
+	if res.StatusCode != 201 {
+		if res.StatusCode == 401 {
+			return "", errLoginRequired
+		}
+		return "", utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d trying to push %s blob", res.StatusCode, imageName), res)
+	}
+
+	type sumReturn struct {
+		Checksum string `json:"checksum"`
+	}
+
+	decoder := json.NewDecoder(res.Body)
+	var sumInfo sumReturn
+
+	err = decoder.Decode(&sumInfo)
+	if err != nil {
+		return "", fmt.Errorf("unable to decode PutV2ImageBlob JSON response: %s", err)
+	}
+
+	// XXX this is a json struct from the registry, with its checksum
+	return sumInfo.Checksum, nil
+}
+
+// Finally Push the (signed) manifest of the blobs we've just pushed
+func (r *Session) PutV2ImageManifest(imageName, tagName string, manifestRdr io.Reader, token []string) error {
+	vars := map[string]string{
+		"imagename": imageName,
+		"tagname":   tagName,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "manifests", vars)
+	if err != nil {
+		return err
+	}
+
+	method := "PUT"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), manifestRdr)
+	if err != nil {
+		return err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return err
+	}
+	res.Body.Close()
+	if res.StatusCode != 201 {
+		if res.StatusCode == 401 {
+			return errLoginRequired
+		}
+		return utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d trying to push %s:%s manifest", res.StatusCode, imageName, tagName), res)
+	}
+
+	return nil
+}
+
+// Given a repository name, returns a json array of string tags
+func (r *Session) GetV2RemoteTags(imageName string, token []string) ([]string, error) {
+	vars := map[string]string{
+		"imagename": imageName,
+	}
+
+	routeURL, err := getV2URL(r.indexEndpoint, "tags", vars)
+	if err != nil {
+		return nil, err
+	}
+
+	method := "GET"
+	log.Debugf("[registry] Calling %q %s", method, routeURL.String())
+
+	req, err := r.reqFactory.NewRequest(method, routeURL.String(), nil)
+	if err != nil {
+		return nil, err
+	}
+	setTokenAuth(req, token)
+	res, _, err := r.doRequest(req)
+	if err != nil {
+		return nil, err
+	}
+	defer res.Body.Close()
+	if res.StatusCode != 200 {
+		if res.StatusCode == 401 {
+			return nil, errLoginRequired
+		} else if res.StatusCode == 404 {
+			return nil, ErrDoesNotExist
+		}
+		return nil, utils.NewHTTPRequestError(fmt.Sprintf("Server error: %d trying to fetch for %s", res.StatusCode, imageName), res)
+	}
+
+	decoder := json.NewDecoder(res.Body)
+	var tags []string
+	err = decoder.Decode(&tags)
+	if err != nil {
+		return nil, fmt.Errorf("Error while decoding the http response: %s", err)
+	}
+	return tags, nil
+}

+ 10 - 2
registry/types.go

@@ -32,6 +32,15 @@ type RegistryInfo struct {
 	Standalone bool   `json:"standalone"`
 }
 
+type ManifestData struct {
+	Name          string   `json:"name"`
+	Tag           string   `json:"tag"`
+	Architecture  string   `json:"architecture"`
+	BlobSums      []string `json:"blobSums"`
+	History       []string `json:"history"`
+	SchemaVersion int      `json:"schemaVersion"`
+}
+
 type APIVersion int
 
 func (av APIVersion) String() string {
@@ -45,7 +54,6 @@ var apiVersions = map[APIVersion]string{
 }
 
 const (
-	_           = iota
-	APIVersion1 = iota
+	APIVersion1 = iota + 1
 	APIVersion2
 )

+ 74 - 0
trust/service.go

@@ -0,0 +1,74 @@
+package trust
+
+import (
+	"fmt"
+	"time"
+
+	"github.com/docker/docker/engine"
+	"github.com/docker/docker/pkg/log"
+	"github.com/docker/libtrust"
+)
+
+func (t *TrustStore) Install(eng *engine.Engine) error {
+	for name, handler := range map[string]engine.Handler{
+		"trust_key_check":   t.CmdCheckKey,
+		"trust_update_base": t.CmdUpdateBase,
+	} {
+		if err := eng.Register(name, handler); err != nil {
+			return fmt.Errorf("Could not register %q: %v", name, err)
+		}
+	}
+	return nil
+}
+
+func (t *TrustStore) CmdCheckKey(job *engine.Job) engine.Status {
+	if n := len(job.Args); n != 1 {
+		return job.Errorf("Usage: %s NAMESPACE", job.Name)
+	}
+	var (
+		namespace = job.Args[0]
+		keyBytes  = job.Getenv("PublicKey")
+	)
+
+	if keyBytes == "" {
+		return job.Errorf("Missing PublicKey")
+	}
+	pk, err := libtrust.UnmarshalPublicKeyJWK([]byte(keyBytes))
+	if err != nil {
+		return job.Errorf("Error unmarshalling public key: %s", err)
+	}
+
+	permission := uint16(job.GetenvInt("Permission"))
+	if permission == 0 {
+		permission = 0x03
+	}
+
+	t.RLock()
+	defer t.RUnlock()
+	if t.graph == nil {
+		job.Stdout.Write([]byte("no graph"))
+		return engine.StatusOK
+	}
+
+	// Check if any expired grants
+	verified, err := t.graph.Verify(pk, namespace, permission)
+	if err != nil {
+		return job.Errorf("Error verifying key to namespace: %s", namespace)
+	}
+	if !verified {
+		log.Debugf("Verification failed for %s using key %s", namespace, pk.KeyID())
+		job.Stdout.Write([]byte("not verified"))
+	} else if t.expiration.Before(time.Now()) {
+		job.Stdout.Write([]byte("expired"))
+	} else {
+		job.Stdout.Write([]byte("verified"))
+	}
+
+	return engine.StatusOK
+}
+
+func (t *TrustStore) CmdUpdateBase(job *engine.Job) engine.Status {
+	t.fetch()
+
+	return engine.StatusOK
+}

+ 199 - 0
trust/trusts.go

@@ -0,0 +1,199 @@
+package trust
+
+import (
+	"crypto/x509"
+	"errors"
+	"io/ioutil"
+	"net/http"
+	"net/url"
+	"os"
+	"path"
+	"path/filepath"
+	"sync"
+	"time"
+
+	"github.com/docker/docker/pkg/log"
+	"github.com/docker/libtrust/trustgraph"
+)
+
+type TrustStore struct {
+	path          string
+	caPool        *x509.CertPool
+	graph         trustgraph.TrustGraph
+	expiration    time.Time
+	fetcher       *time.Timer
+	fetchTime     time.Duration
+	autofetch     bool
+	httpClient    *http.Client
+	baseEndpoints map[string]*url.URL
+
+	sync.RWMutex
+}
+
+// defaultFetchtime represents the starting duration to wait between
+// fetching sections of the graph.  Unsuccessful fetches should
+// increase time between fetching.
+const defaultFetchtime = 45 * time.Second
+
+var baseEndpoints = map[string]string{"official": "https://dvjy3tqbc323p.cloudfront.net/trust/official.json"}
+
+func NewTrustStore(path string) (*TrustStore, error) {
+	abspath, err := filepath.Abs(path)
+	if err != nil {
+		return nil, err
+	}
+
+	// Create base graph url map
+	endpoints := map[string]*url.URL{}
+	for name, endpoint := range baseEndpoints {
+		u, err := url.Parse(endpoint)
+		if err != nil {
+			return nil, err
+		}
+		endpoints[name] = u
+	}
+
+	// Load grant files
+	t := &TrustStore{
+		path:          abspath,
+		caPool:        nil,
+		httpClient:    &http.Client{},
+		fetchTime:     time.Millisecond,
+		baseEndpoints: endpoints,
+	}
+
+	err = t.reload()
+	if err != nil {
+		return nil, err
+	}
+
+	return t, nil
+}
+
+func (t *TrustStore) reload() error {
+	t.Lock()
+	defer t.Unlock()
+
+	matches, err := filepath.Glob(filepath.Join(t.path, "*.json"))
+	if err != nil {
+		return err
+	}
+	statements := make([]*trustgraph.Statement, len(matches))
+	for i, match := range matches {
+		f, err := os.Open(match)
+		if err != nil {
+			return err
+		}
+		statements[i], err = trustgraph.LoadStatement(f, nil)
+		if err != nil {
+			f.Close()
+			return err
+		}
+		f.Close()
+	}
+	if len(statements) == 0 {
+		if t.autofetch {
+			log.Debugf("No grants, fetching")
+			t.fetcher = time.AfterFunc(t.fetchTime, t.fetch)
+		}
+		return nil
+	}
+
+	grants, expiration, err := trustgraph.CollapseStatements(statements, true)
+	if err != nil {
+		return err
+	}
+
+	t.expiration = expiration
+	t.graph = trustgraph.NewMemoryGraph(grants)
+	log.Debugf("Reloaded graph with %d grants expiring at %s", len(grants), expiration)
+
+	if t.autofetch {
+		nextFetch := expiration.Sub(time.Now())
+		if nextFetch < 0 {
+			nextFetch = defaultFetchtime
+		} else {
+			nextFetch = time.Duration(0.8 * (float64)(nextFetch))
+		}
+		t.fetcher = time.AfterFunc(nextFetch, t.fetch)
+	}
+
+	return nil
+}
+
+func (t *TrustStore) fetchBaseGraph(u *url.URL) (*trustgraph.Statement, error) {
+	req := &http.Request{
+		Method:     "GET",
+		URL:        u,
+		Proto:      "HTTP/1.1",
+		ProtoMajor: 1,
+		ProtoMinor: 1,
+		Header:     make(http.Header),
+		Body:       nil,
+		Host:       u.Host,
+	}
+
+	resp, err := t.httpClient.Do(req)
+	if err != nil {
+		return nil, err
+	}
+	if resp.StatusCode == 404 {
+		return nil, errors.New("base graph does not exist")
+	}
+
+	defer resp.Body.Close()
+
+	return trustgraph.LoadStatement(resp.Body, t.caPool)
+}
+
+// fetch retrieves updated base graphs.  This function cannot error, it
+// should only log errors
+func (t *TrustStore) fetch() {
+	t.Lock()
+	defer t.Unlock()
+
+	if t.autofetch && t.fetcher == nil {
+		// Do nothing ??
+		return
+	}
+
+	fetchCount := 0
+	for bg, ep := range t.baseEndpoints {
+		statement, err := t.fetchBaseGraph(ep)
+		if err != nil {
+			log.Infof("Trust graph fetch failed: %s", err)
+			continue
+		}
+		b, err := statement.Bytes()
+		if err != nil {
+			log.Infof("Bad trust graph statement: %s", err)
+			continue
+		}
+		// TODO check if value differs
+		err = ioutil.WriteFile(path.Join(t.path, bg+".json"), b, 0600)
+		if err != nil {
+			log.Infof("Error writing trust graph statement: %s", err)
+		}
+		fetchCount++
+	}
+	log.Debugf("Fetched %d base graphs at %s", fetchCount, time.Now())
+
+	if fetchCount > 0 {
+		go func() {
+			err := t.reload()
+			if err != nil {
+				// TODO log
+				log.Infof("Reload of trust graph failed: %s", err)
+			}
+		}()
+		t.fetchTime = defaultFetchtime
+		t.fetcher = nil
+	} else if t.autofetch {
+		maxTime := 10 * defaultFetchtime
+		t.fetchTime = time.Duration(1.5 * (float64)(t.fetchTime+time.Second))
+		if t.fetchTime > maxTime {
+			t.fetchTime = maxTime
+		}
+		t.fetcher = time.AfterFunc(t.fetchTime, t.fetch)
+	}
+}