blob: 2cfede9b288acbe70551bff804680244229e8caf [file] [log] [blame]
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
/*
* Protocol Buffers describing the Artifact API, for communicating with a runner
* for artifact staging and retrieval over GRPC.
*/
syntax = "proto3";
package org.apache.beam.model.job_management.v1;
option go_package = "jobmanagement_v1";
option java_package = "org.apache.beam.model.jobmanagement.v1";
option java_outer_classname = "ArtifactApi";
import "beam_runner_api.proto";
// A service to stage artifacts for use in a Job.
service ArtifactStagingService {
// Stage an artifact to be available during job execution. The first request must contain the
// name of the artifact. All future requests must contain sequential chunks of the content of
// the artifact.
rpc PutArtifact(stream PutArtifactRequest) returns (PutArtifactResponse);
// Commit the manifest for a Job. All artifacts must have been successfully uploaded
// before this call is made.
//
// Throws error INVALID_ARGUMENT if not all of the members of the manifest are present
rpc CommitManifest(CommitManifestRequest) returns (CommitManifestResponse);
}
// A service to retrieve artifacts for use in a Job.
service ArtifactRetrievalService {
// Get the manifest for the job
rpc GetManifest(GetManifestRequest) returns (GetManifestResponse);
// Get an artifact staged for the job. The requested artifact must be within the manifest
rpc GetArtifact(GetArtifactRequest) returns (stream ArtifactChunk);
}
// An artifact identifier and associated metadata.
message ArtifactMetadata {
// (Required) The name of the artifact.
string name = 1;
// (Optional) The Unix-like permissions of the artifact
uint32 permissions = 2;
// (Optional) The hex-encoded sha256 checksum of the artifact. Used, among other things, by
// harness boot code to validate the integrity of the artifact.
string sha256 = 4;
}
// A collection of artifacts.
message Manifest {
repeated ArtifactMetadata artifact = 1;
}
// A manifest with location information.
message ProxyManifest {
Manifest manifest = 1;
message Location {
string name = 1;
string uri = 2;
}
repeated Location location = 2;
}
// A request to get the manifest of a Job.
message GetManifestRequest {
// (Required) An opaque token representing the entirety of the staged artifacts.
// Returned in CommitManifestResponse.
string retrieval_token = 1;
}
// A response containing a job manifest.
message GetManifestResponse {
Manifest manifest = 1;
}
// A request to get an artifact. The artifact must be present in the manifest for the job.
message GetArtifactRequest {
// (Required) The name of the artifact to retrieve.
string name = 1;
// (Required) An opaque token representing the entirety of the staged artifacts.
// Returned in CommitManifestResponse.
string retrieval_token = 2;
}
// Part of an artifact.
message ArtifactChunk {
bytes data = 1;
}
message PutArtifactMetadata {
// (Required) A token for artifact staging session. This token can be obtained
// from PrepareJob request in JobService
string staging_session_token = 1;
// (Required) The Artifact metadata.
ArtifactMetadata metadata = 2;
}
// A request to stage an artifact.
message PutArtifactRequest {
// (Required)
oneof content {
// The first message in a PutArtifact call must contain this field.
PutArtifactMetadata metadata = 1;
// A chunk of the artifact. All messages after the first in a PutArtifact call must contain a
// chunk.
ArtifactChunk data = 2;
}
}
message PutArtifactResponse {
}
// A request to commit the manifest for a Job. All artifacts must have been successfully uploaded
// before this call is made.
message CommitManifestRequest {
// (Required) The manifest to commit.
Manifest manifest = 1;
// (Required) A token for artifact staging session. This token can be obtained
// from PrepareJob request in JobService
string staging_session_token = 2;
}
// The result of committing a manifest.
message CommitManifestResponse {
enum Constants {
// Token indicating that no artifacts were staged and therefore no retrieval attempt is necessary.
NO_ARTIFACTS_STAGED_TOKEN = 0 [(org.apache.beam.model.pipeline.v1.beam_constant) = "__no_artifacts_staged__"];
}
// (Required) An opaque token representing the entirety of the staged artifacts.
// This can be used to retrieve the manifest and artifacts from an associated
// ArtifactRetrievalService.
string retrieval_token = 1;
}