jobmanagement_v1

package
v2.3.0+incompatible Latest Latest
Warning

This package is not in the latest version of its module.

Go to latest
Published: Feb 10, 2018 License: Apache-2.0 Imports: 7 Imported by: 0

Documentation

Overview

Package jobmanagement_v1 is a generated protocol buffer package.

It is generated from these files:

beam_job_api.proto
beam_artifact_api.proto

It has these top-level messages:

PrepareJobRequest
PrepareJobResponse
RunJobRequest
RunJobResponse
CancelJobRequest
CancelJobResponse
GetJobStateRequest
GetJobStateResponse
JobMessagesRequest
JobMessage
JobMessagesResponse
JobState
ArtifactMetadata
Manifest
ProxyManifest
GetManifestRequest
GetManifestResponse
GetArtifactRequest
ArtifactChunk
PutArtifactRequest
PutArtifactResponse
CommitManifestRequest
CommitManifestResponse

Index

Constants

This section is empty.

Variables

View Source
var JobMessage_MessageImportance_name = map[int32]string{
	0: "MESSAGE_IMPORTANCE_UNSPECIFIED",
	1: "JOB_MESSAGE_DEBUG",
	2: "JOB_MESSAGE_DETAILED",
	3: "JOB_MESSAGE_BASIC",
	4: "JOB_MESSAGE_WARNING",
	5: "JOB_MESSAGE_ERROR",
}
View Source
var JobMessage_MessageImportance_value = map[string]int32{
	"MESSAGE_IMPORTANCE_UNSPECIFIED": 0,
	"JOB_MESSAGE_DEBUG":              1,
	"JOB_MESSAGE_DETAILED":           2,
	"JOB_MESSAGE_BASIC":              3,
	"JOB_MESSAGE_WARNING":            4,
	"JOB_MESSAGE_ERROR":              5,
}
View Source
var JobState_Enum_name = map[int32]string{
	0:  "UNSPECIFIED",
	1:  "STOPPED",
	2:  "RUNNING",
	3:  "DONE",
	4:  "FAILED",
	5:  "CANCELLED",
	6:  "UPDATED",
	7:  "DRAINING",
	8:  "DRAINED",
	9:  "STARTING",
	10: "CANCELLING",
}
View Source
var JobState_Enum_value = map[string]int32{
	"UNSPECIFIED": 0,
	"STOPPED":     1,
	"RUNNING":     2,
	"DONE":        3,
	"FAILED":      4,
	"CANCELLED":   5,
	"UPDATED":     6,
	"DRAINING":    7,
	"DRAINED":     8,
	"STARTING":    9,
	"CANCELLING":  10,
}

Functions

func RegisterArtifactRetrievalServiceServer

func RegisterArtifactRetrievalServiceServer(s *grpc.Server, srv ArtifactRetrievalServiceServer)

func RegisterArtifactStagingServiceServer

func RegisterArtifactStagingServiceServer(s *grpc.Server, srv ArtifactStagingServiceServer)

func RegisterJobServiceServer

func RegisterJobServiceServer(s *grpc.Server, srv JobServiceServer)

Types

type ArtifactChunk

type ArtifactChunk struct {
	Data []byte `protobuf:"bytes,1,opt,name=data,proto3" json:"data,omitempty"`
}

Part of an artifact.

func (*ArtifactChunk) Descriptor

func (*ArtifactChunk) Descriptor() ([]byte, []int)

func (*ArtifactChunk) GetData

func (m *ArtifactChunk) GetData() []byte

func (*ArtifactChunk) ProtoMessage

func (*ArtifactChunk) ProtoMessage()

func (*ArtifactChunk) Reset

func (m *ArtifactChunk) Reset()

func (*ArtifactChunk) String

func (m *ArtifactChunk) String() string

type ArtifactMetadata

type ArtifactMetadata struct {
	// (Required) The name of the artifact.
	Name string `protobuf:"bytes,1,opt,name=name" json:"name,omitempty"`
	// (Optional) The Unix-like permissions of the artifact
	Permissions uint32 `protobuf:"varint,2,opt,name=permissions" json:"permissions,omitempty"`
	// (Optional) The base64-encoded md5 checksum of the artifact. Used, among other things, by
	// harness boot code to validate the integrity of the artifact.
	Md5 string `protobuf:"bytes,3,opt,name=md5" json:"md5,omitempty"`
}

An artifact identifier and associated metadata.

func (*ArtifactMetadata) Descriptor

func (*ArtifactMetadata) Descriptor() ([]byte, []int)

func (*ArtifactMetadata) GetMd5

func (m *ArtifactMetadata) GetMd5() string

func (*ArtifactMetadata) GetName

func (m *ArtifactMetadata) GetName() string

func (*ArtifactMetadata) GetPermissions

func (m *ArtifactMetadata) GetPermissions() uint32

func (*ArtifactMetadata) ProtoMessage

func (*ArtifactMetadata) ProtoMessage()

func (*ArtifactMetadata) Reset

func (m *ArtifactMetadata) Reset()

func (*ArtifactMetadata) String

func (m *ArtifactMetadata) String() string

type ArtifactRetrievalServiceClient

type ArtifactRetrievalServiceClient interface {
	// Get the manifest for the job
	GetManifest(ctx context.Context, in *GetManifestRequest, opts ...grpc.CallOption) (*GetManifestResponse, error)
	// Get an artifact staged for the job. The requested artifact must be within the manifest
	GetArtifact(ctx context.Context, in *GetArtifactRequest, opts ...grpc.CallOption) (ArtifactRetrievalService_GetArtifactClient, error)
}

func NewArtifactRetrievalServiceClient

func NewArtifactRetrievalServiceClient(cc *grpc.ClientConn) ArtifactRetrievalServiceClient

type ArtifactRetrievalServiceServer

type ArtifactRetrievalServiceServer interface {
	// Get the manifest for the job
	GetManifest(context.Context, *GetManifestRequest) (*GetManifestResponse, error)
	// Get an artifact staged for the job. The requested artifact must be within the manifest
	GetArtifact(*GetArtifactRequest, ArtifactRetrievalService_GetArtifactServer) error
}

type ArtifactRetrievalService_GetArtifactClient

type ArtifactRetrievalService_GetArtifactClient interface {
	Recv() (*ArtifactChunk, error)
	grpc.ClientStream
}

type ArtifactRetrievalService_GetArtifactServer

type ArtifactRetrievalService_GetArtifactServer interface {
	Send(*ArtifactChunk) error
	grpc.ServerStream
}

type ArtifactStagingServiceClient

type ArtifactStagingServiceClient interface {
	// 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.
	PutArtifact(ctx context.Context, opts ...grpc.CallOption) (ArtifactStagingService_PutArtifactClient, error)
	// 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
	CommitManifest(ctx context.Context, in *CommitManifestRequest, opts ...grpc.CallOption) (*CommitManifestResponse, error)
}

func NewArtifactStagingServiceClient

func NewArtifactStagingServiceClient(cc *grpc.ClientConn) ArtifactStagingServiceClient

type ArtifactStagingServiceServer

type ArtifactStagingServiceServer interface {
	// 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.
	PutArtifact(ArtifactStagingService_PutArtifactServer) error
	// 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
	CommitManifest(context.Context, *CommitManifestRequest) (*CommitManifestResponse, error)
}

type ArtifactStagingService_PutArtifactClient

type ArtifactStagingService_PutArtifactClient interface {
	Send(*PutArtifactRequest) error
	CloseAndRecv() (*PutArtifactResponse, error)
	grpc.ClientStream
}

type ArtifactStagingService_PutArtifactServer

type ArtifactStagingService_PutArtifactServer interface {
	SendAndClose(*PutArtifactResponse) error
	Recv() (*PutArtifactRequest, error)
	grpc.ServerStream
}

type CancelJobRequest

type CancelJobRequest struct {
	JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId" json:"job_id,omitempty"`
}

Cancel is a synchronus request that returns a job state back Throws error GRPC_STATUS_UNAVAILABLE if server is down Throws error NOT_FOUND if the jobId is not found

func (*CancelJobRequest) Descriptor

func (*CancelJobRequest) Descriptor() ([]byte, []int)

func (*CancelJobRequest) GetJobId

func (m *CancelJobRequest) GetJobId() string

func (*CancelJobRequest) ProtoMessage

func (*CancelJobRequest) ProtoMessage()

func (*CancelJobRequest) Reset

func (m *CancelJobRequest) Reset()

func (*CancelJobRequest) String

func (m *CancelJobRequest) String() string

type CancelJobResponse

type CancelJobResponse struct {
	State JobState_Enum `protobuf:"varint,1,opt,name=state,enum=org.apache.beam.model.job_management.v1.JobState_Enum" json:"state,omitempty"`
}

Valid responses include any terminal state or CANCELLING

func (*CancelJobResponse) Descriptor

func (*CancelJobResponse) Descriptor() ([]byte, []int)

func (*CancelJobResponse) GetState

func (m *CancelJobResponse) GetState() JobState_Enum

func (*CancelJobResponse) ProtoMessage

func (*CancelJobResponse) ProtoMessage()

func (*CancelJobResponse) Reset

func (m *CancelJobResponse) Reset()

func (*CancelJobResponse) String

func (m *CancelJobResponse) String() string

type CommitManifestRequest

type CommitManifestRequest struct {
	// (Required) The manifest to commit.
	Manifest *Manifest `protobuf:"bytes,1,opt,name=manifest" json:"manifest,omitempty"`
}

A request to commit the manifest for a Job. All artifacts must have been successfully uploaded before this call is made.

func (*CommitManifestRequest) Descriptor

func (*CommitManifestRequest) Descriptor() ([]byte, []int)

func (*CommitManifestRequest) GetManifest

func (m *CommitManifestRequest) GetManifest() *Manifest

func (*CommitManifestRequest) ProtoMessage

func (*CommitManifestRequest) ProtoMessage()

func (*CommitManifestRequest) Reset

func (m *CommitManifestRequest) Reset()

func (*CommitManifestRequest) String

func (m *CommitManifestRequest) String() string

type CommitManifestResponse

type CommitManifestResponse struct {
	// (Required) An opaque token representing the entirety of the staged artifacts.
	StagingToken string `protobuf:"bytes,1,opt,name=staging_token,json=stagingToken" json:"staging_token,omitempty"`
}

The result of committing a manifest.

func (*CommitManifestResponse) Descriptor

func (*CommitManifestResponse) Descriptor() ([]byte, []int)

func (*CommitManifestResponse) GetStagingToken

func (m *CommitManifestResponse) GetStagingToken() string

func (*CommitManifestResponse) ProtoMessage

func (*CommitManifestResponse) ProtoMessage()

func (*CommitManifestResponse) Reset

func (m *CommitManifestResponse) Reset()

func (*CommitManifestResponse) String

func (m *CommitManifestResponse) String() string

type GetArtifactRequest

type GetArtifactRequest struct {
	// (Required) The name of the artifact to retrieve.
	Name string `protobuf:"bytes,1,opt,name=name" json:"name,omitempty"`
}

A request to get an artifact. The artifact must be present in the manifest for the job.

func (*GetArtifactRequest) Descriptor

func (*GetArtifactRequest) Descriptor() ([]byte, []int)

func (*GetArtifactRequest) GetName

func (m *GetArtifactRequest) GetName() string

func (*GetArtifactRequest) ProtoMessage

func (*GetArtifactRequest) ProtoMessage()

func (*GetArtifactRequest) Reset

func (m *GetArtifactRequest) Reset()

func (*GetArtifactRequest) String

func (m *GetArtifactRequest) String() string

type GetJobStateRequest

type GetJobStateRequest struct {
	JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId" json:"job_id,omitempty"`
}

GetState is a synchronus request that returns a job state back Throws error GRPC_STATUS_UNAVAILABLE if server is down Throws error NOT_FOUND if the jobId is not found

func (*GetJobStateRequest) Descriptor

func (*GetJobStateRequest) Descriptor() ([]byte, []int)

func (*GetJobStateRequest) GetJobId

func (m *GetJobStateRequest) GetJobId() string

func (*GetJobStateRequest) ProtoMessage

func (*GetJobStateRequest) ProtoMessage()

func (*GetJobStateRequest) Reset

func (m *GetJobStateRequest) Reset()

func (*GetJobStateRequest) String

func (m *GetJobStateRequest) String() string

type GetJobStateResponse

type GetJobStateResponse struct {
	State JobState_Enum `protobuf:"varint,1,opt,name=state,enum=org.apache.beam.model.job_management.v1.JobState_Enum" json:"state,omitempty"`
}

func (*GetJobStateResponse) Descriptor

func (*GetJobStateResponse) Descriptor() ([]byte, []int)

func (*GetJobStateResponse) GetState

func (m *GetJobStateResponse) GetState() JobState_Enum

func (*GetJobStateResponse) ProtoMessage

func (*GetJobStateResponse) ProtoMessage()

func (*GetJobStateResponse) Reset

func (m *GetJobStateResponse) Reset()

func (*GetJobStateResponse) String

func (m *GetJobStateResponse) String() string

type GetManifestRequest

type GetManifestRequest struct {
}

A request to get the manifest of a Job.

func (*GetManifestRequest) Descriptor

func (*GetManifestRequest) Descriptor() ([]byte, []int)

func (*GetManifestRequest) ProtoMessage

func (*GetManifestRequest) ProtoMessage()

func (*GetManifestRequest) Reset

func (m *GetManifestRequest) Reset()

func (*GetManifestRequest) String

func (m *GetManifestRequest) String() string

type GetManifestResponse

type GetManifestResponse struct {
	Manifest *Manifest `protobuf:"bytes,1,opt,name=manifest" json:"manifest,omitempty"`
}

A response containing a job manifest.

func (*GetManifestResponse) Descriptor

func (*GetManifestResponse) Descriptor() ([]byte, []int)

func (*GetManifestResponse) GetManifest

func (m *GetManifestResponse) GetManifest() *Manifest

func (*GetManifestResponse) ProtoMessage

func (*GetManifestResponse) ProtoMessage()

func (*GetManifestResponse) Reset

func (m *GetManifestResponse) Reset()

func (*GetManifestResponse) String

func (m *GetManifestResponse) String() string

type JobMessage

type JobMessage struct {
	MessageId   string                       `protobuf:"bytes,1,opt,name=message_id,json=messageId" json:"message_id,omitempty"`
	Time        string                       `protobuf:"bytes,2,opt,name=time" json:"time,omitempty"`
	Importance  JobMessage_MessageImportance `` /* 141-byte string literal not displayed */
	MessageText string                       `protobuf:"bytes,4,opt,name=message_text,json=messageText" json:"message_text,omitempty"`
}

func (*JobMessage) Descriptor

func (*JobMessage) Descriptor() ([]byte, []int)

func (*JobMessage) GetImportance

func (m *JobMessage) GetImportance() JobMessage_MessageImportance

func (*JobMessage) GetMessageId

func (m *JobMessage) GetMessageId() string

func (*JobMessage) GetMessageText

func (m *JobMessage) GetMessageText() string

func (*JobMessage) GetTime

func (m *JobMessage) GetTime() string

func (*JobMessage) ProtoMessage

func (*JobMessage) ProtoMessage()

func (*JobMessage) Reset

func (m *JobMessage) Reset()

func (*JobMessage) String

func (m *JobMessage) String() string

type JobMessage_MessageImportance

type JobMessage_MessageImportance int32
const (
	JobMessage_MESSAGE_IMPORTANCE_UNSPECIFIED JobMessage_MessageImportance = 0
	JobMessage_JOB_MESSAGE_DEBUG              JobMessage_MessageImportance = 1
	JobMessage_JOB_MESSAGE_DETAILED           JobMessage_MessageImportance = 2
	JobMessage_JOB_MESSAGE_BASIC              JobMessage_MessageImportance = 3
	JobMessage_JOB_MESSAGE_WARNING            JobMessage_MessageImportance = 4
	JobMessage_JOB_MESSAGE_ERROR              JobMessage_MessageImportance = 5
)

func (JobMessage_MessageImportance) EnumDescriptor

func (JobMessage_MessageImportance) EnumDescriptor() ([]byte, []int)

func (JobMessage_MessageImportance) String

type JobMessagesRequest

type JobMessagesRequest struct {
	JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId" json:"job_id,omitempty"`
}

GetJobMessages is a streaming api for streaming job messages from the service One request will connect you to the job and you'll get a stream of job state and job messages back; one is used for logging and the other for detecting the job ended.

func (*JobMessagesRequest) Descriptor

func (*JobMessagesRequest) Descriptor() ([]byte, []int)

func (*JobMessagesRequest) GetJobId

func (m *JobMessagesRequest) GetJobId() string

func (*JobMessagesRequest) ProtoMessage

func (*JobMessagesRequest) ProtoMessage()

func (*JobMessagesRequest) Reset

func (m *JobMessagesRequest) Reset()

func (*JobMessagesRequest) String

func (m *JobMessagesRequest) String() string

type JobMessagesResponse

type JobMessagesResponse struct {
	// Types that are valid to be assigned to Response:
	//	*JobMessagesResponse_MessageResponse
	//	*JobMessagesResponse_StateResponse
	Response isJobMessagesResponse_Response `protobuf_oneof:"response"`
}

func (*JobMessagesResponse) Descriptor

func (*JobMessagesResponse) Descriptor() ([]byte, []int)

func (*JobMessagesResponse) GetMessageResponse

func (m *JobMessagesResponse) GetMessageResponse() *JobMessage

func (*JobMessagesResponse) GetResponse

func (m *JobMessagesResponse) GetResponse() isJobMessagesResponse_Response

func (*JobMessagesResponse) GetStateResponse

func (m *JobMessagesResponse) GetStateResponse() *GetJobStateResponse

func (*JobMessagesResponse) ProtoMessage

func (*JobMessagesResponse) ProtoMessage()

func (*JobMessagesResponse) Reset

func (m *JobMessagesResponse) Reset()

func (*JobMessagesResponse) String

func (m *JobMessagesResponse) String() string

func (*JobMessagesResponse) XXX_OneofFuncs

func (*JobMessagesResponse) XXX_OneofFuncs() (func(msg proto.Message, b *proto.Buffer) error, func(msg proto.Message, tag, wire int, b *proto.Buffer) (bool, error), func(msg proto.Message) (n int), []interface{})

XXX_OneofFuncs is for the internal use of the proto package.

type JobMessagesResponse_MessageResponse

type JobMessagesResponse_MessageResponse struct {
	MessageResponse *JobMessage `protobuf:"bytes,1,opt,name=message_response,json=messageResponse,oneof"`
}

type JobMessagesResponse_StateResponse

type JobMessagesResponse_StateResponse struct {
	StateResponse *GetJobStateResponse `protobuf:"bytes,2,opt,name=state_response,json=stateResponse,oneof"`
}

type JobServiceClient

type JobServiceClient interface {
	// Prepare a job for execution. The job will not be executed until a call is made to run with the
	// returned preparationId.
	Prepare(ctx context.Context, in *PrepareJobRequest, opts ...grpc.CallOption) (*PrepareJobResponse, error)
	// Submit the job for execution
	Run(ctx context.Context, in *RunJobRequest, opts ...grpc.CallOption) (*RunJobResponse, error)
	// Get the current state of the job
	GetState(ctx context.Context, in *GetJobStateRequest, opts ...grpc.CallOption) (*GetJobStateResponse, error)
	// Cancel the job
	Cancel(ctx context.Context, in *CancelJobRequest, opts ...grpc.CallOption) (*CancelJobResponse, error)
	// Subscribe to a stream of state changes of the job, will immediately return the current state of the job as the first response.
	GetStateStream(ctx context.Context, in *GetJobStateRequest, opts ...grpc.CallOption) (JobService_GetStateStreamClient, error)
	// Subscribe to a stream of state changes and messages from the job
	GetMessageStream(ctx context.Context, in *JobMessagesRequest, opts ...grpc.CallOption) (JobService_GetMessageStreamClient, error)
}

func NewJobServiceClient

func NewJobServiceClient(cc *grpc.ClientConn) JobServiceClient

type JobServiceServer

type JobServiceServer interface {
	// Prepare a job for execution. The job will not be executed until a call is made to run with the
	// returned preparationId.
	Prepare(context.Context, *PrepareJobRequest) (*PrepareJobResponse, error)
	// Submit the job for execution
	Run(context.Context, *RunJobRequest) (*RunJobResponse, error)
	// Get the current state of the job
	GetState(context.Context, *GetJobStateRequest) (*GetJobStateResponse, error)
	// Cancel the job
	Cancel(context.Context, *CancelJobRequest) (*CancelJobResponse, error)
	// Subscribe to a stream of state changes of the job, will immediately return the current state of the job as the first response.
	GetStateStream(*GetJobStateRequest, JobService_GetStateStreamServer) error
	// Subscribe to a stream of state changes and messages from the job
	GetMessageStream(*JobMessagesRequest, JobService_GetMessageStreamServer) error
}

type JobService_GetMessageStreamClient

type JobService_GetMessageStreamClient interface {
	Recv() (*JobMessagesResponse, error)
	grpc.ClientStream
}

type JobService_GetMessageStreamServer

type JobService_GetMessageStreamServer interface {
	Send(*JobMessagesResponse) error
	grpc.ServerStream
}

type JobService_GetStateStreamClient

type JobService_GetStateStreamClient interface {
	Recv() (*GetJobStateResponse, error)
	grpc.ClientStream
}

type JobService_GetStateStreamServer

type JobService_GetStateStreamServer interface {
	Send(*GetJobStateResponse) error
	grpc.ServerStream
}

type JobState

type JobState struct {
}

Enumeration of all JobStates

func (*JobState) Descriptor

func (*JobState) Descriptor() ([]byte, []int)

func (*JobState) ProtoMessage

func (*JobState) ProtoMessage()

func (*JobState) Reset

func (m *JobState) Reset()

func (*JobState) String

func (m *JobState) String() string

type JobState_Enum

type JobState_Enum int32
const (
	JobState_UNSPECIFIED JobState_Enum = 0
	JobState_STOPPED     JobState_Enum = 1
	JobState_RUNNING     JobState_Enum = 2
	JobState_DONE        JobState_Enum = 3
	JobState_FAILED      JobState_Enum = 4
	JobState_CANCELLED   JobState_Enum = 5
	JobState_UPDATED     JobState_Enum = 6
	JobState_DRAINING    JobState_Enum = 7
	JobState_DRAINED     JobState_Enum = 8
	JobState_STARTING    JobState_Enum = 9
	JobState_CANCELLING  JobState_Enum = 10
)

func (JobState_Enum) EnumDescriptor

func (JobState_Enum) EnumDescriptor() ([]byte, []int)

func (JobState_Enum) String

func (x JobState_Enum) String() string

type Manifest

type Manifest struct {
	Artifact []*ArtifactMetadata `protobuf:"bytes,1,rep,name=artifact" json:"artifact,omitempty"`
}

A collection of artifacts.

func (*Manifest) Descriptor

func (*Manifest) Descriptor() ([]byte, []int)

func (*Manifest) GetArtifact

func (m *Manifest) GetArtifact() []*ArtifactMetadata

func (*Manifest) ProtoMessage

func (*Manifest) ProtoMessage()

func (*Manifest) Reset

func (m *Manifest) Reset()

func (*Manifest) String

func (m *Manifest) String() string

type PrepareJobRequest

type PrepareJobRequest struct {
	Pipeline        *org_apache_beam_model_pipeline_v1.Pipeline `protobuf:"bytes,1,opt,name=pipeline" json:"pipeline,omitempty"`
	PipelineOptions *google_protobuf1.Struct                    `protobuf:"bytes,2,opt,name=pipeline_options,json=pipelineOptions" json:"pipeline_options,omitempty"`
	JobName         string                                      `protobuf:"bytes,3,opt,name=job_name,json=jobName" json:"job_name,omitempty"`
}

Prepare is a synchronous request that returns a preparationId back Throws error GRPC_STATUS_UNAVAILABLE if server is down Throws error ALREADY_EXISTS if the jobName is reused. Runners are permitted to deduplicate based on the name of the job. Throws error UNKNOWN for all other issues

func (*PrepareJobRequest) Descriptor

func (*PrepareJobRequest) Descriptor() ([]byte, []int)

func (*PrepareJobRequest) GetJobName

func (m *PrepareJobRequest) GetJobName() string

func (*PrepareJobRequest) GetPipeline

func (*PrepareJobRequest) GetPipelineOptions

func (m *PrepareJobRequest) GetPipelineOptions() *google_protobuf1.Struct

func (*PrepareJobRequest) ProtoMessage

func (*PrepareJobRequest) ProtoMessage()

func (*PrepareJobRequest) Reset

func (m *PrepareJobRequest) Reset()

func (*PrepareJobRequest) String

func (m *PrepareJobRequest) String() string

type PrepareJobResponse

type PrepareJobResponse struct {
	// (required) The ID used to associate calls made while preparing the job. preparationId is used
	// to run the job, as well as in other pre-execution APIs such as Artifact staging.
	PreparationId string `protobuf:"bytes,1,opt,name=preparation_id,json=preparationId" json:"preparation_id,omitempty"`
	// An endpoint which exposes the Beam Artifact Staging API. Artifacts used by the job should be
	// staged to this endpoint, and will be available during job execution.
	ArtifactStagingEndpoint *org_apache_beam_model_pipeline_v11.ApiServiceDescriptor `protobuf:"bytes,2,opt,name=artifact_staging_endpoint,json=artifactStagingEndpoint" json:"artifact_staging_endpoint,omitempty"`
}

func (*PrepareJobResponse) Descriptor

func (*PrepareJobResponse) Descriptor() ([]byte, []int)

func (*PrepareJobResponse) GetArtifactStagingEndpoint

func (*PrepareJobResponse) GetPreparationId

func (m *PrepareJobResponse) GetPreparationId() string

func (*PrepareJobResponse) ProtoMessage

func (*PrepareJobResponse) ProtoMessage()

func (*PrepareJobResponse) Reset

func (m *PrepareJobResponse) Reset()

func (*PrepareJobResponse) String

func (m *PrepareJobResponse) String() string

type ProxyManifest

type ProxyManifest struct {
	Manifest *Manifest                 `protobuf:"bytes,1,opt,name=manifest" json:"manifest,omitempty"`
	Location []*ProxyManifest_Location `protobuf:"bytes,2,rep,name=location" json:"location,omitempty"`
}

A manifest with location information.

func (*ProxyManifest) Descriptor

func (*ProxyManifest) Descriptor() ([]byte, []int)

func (*ProxyManifest) GetLocation

func (m *ProxyManifest) GetLocation() []*ProxyManifest_Location

func (*ProxyManifest) GetManifest

func (m *ProxyManifest) GetManifest() *Manifest

func (*ProxyManifest) ProtoMessage

func (*ProxyManifest) ProtoMessage()

func (*ProxyManifest) Reset

func (m *ProxyManifest) Reset()

func (*ProxyManifest) String

func (m *ProxyManifest) String() string

type ProxyManifest_Location

type ProxyManifest_Location struct {
	Name string `protobuf:"bytes,1,opt,name=name" json:"name,omitempty"`
	Uri  string `protobuf:"bytes,2,opt,name=uri" json:"uri,omitempty"`
}

func (*ProxyManifest_Location) Descriptor

func (*ProxyManifest_Location) Descriptor() ([]byte, []int)

func (*ProxyManifest_Location) GetName

func (m *ProxyManifest_Location) GetName() string

func (*ProxyManifest_Location) GetUri

func (m *ProxyManifest_Location) GetUri() string

func (*ProxyManifest_Location) ProtoMessage

func (*ProxyManifest_Location) ProtoMessage()

func (*ProxyManifest_Location) Reset

func (m *ProxyManifest_Location) Reset()

func (*ProxyManifest_Location) String

func (m *ProxyManifest_Location) String() string

type PutArtifactRequest

type PutArtifactRequest struct {
	// (Required)
	//
	// Types that are valid to be assigned to Content:
	//	*PutArtifactRequest_Metadata
	//	*PutArtifactRequest_Data
	Content isPutArtifactRequest_Content `protobuf_oneof:"content"`
}

A request to stage an artifact.

func (*PutArtifactRequest) Descriptor

func (*PutArtifactRequest) Descriptor() ([]byte, []int)

func (*PutArtifactRequest) GetContent

func (m *PutArtifactRequest) GetContent() isPutArtifactRequest_Content

func (*PutArtifactRequest) GetData

func (m *PutArtifactRequest) GetData() *ArtifactChunk

func (*PutArtifactRequest) GetMetadata

func (m *PutArtifactRequest) GetMetadata() *ArtifactMetadata

func (*PutArtifactRequest) ProtoMessage

func (*PutArtifactRequest) ProtoMessage()

func (*PutArtifactRequest) Reset

func (m *PutArtifactRequest) Reset()

func (*PutArtifactRequest) String

func (m *PutArtifactRequest) String() string

func (*PutArtifactRequest) XXX_OneofFuncs

func (*PutArtifactRequest) XXX_OneofFuncs() (func(msg proto.Message, b *proto.Buffer) error, func(msg proto.Message, tag, wire int, b *proto.Buffer) (bool, error), func(msg proto.Message) (n int), []interface{})

XXX_OneofFuncs is for the internal use of the proto package.

type PutArtifactRequest_Data

type PutArtifactRequest_Data struct {
	Data *ArtifactChunk `protobuf:"bytes,2,opt,name=data,oneof"`
}

type PutArtifactRequest_Metadata

type PutArtifactRequest_Metadata struct {
	Metadata *ArtifactMetadata `protobuf:"bytes,1,opt,name=metadata,oneof"`
}

type PutArtifactResponse

type PutArtifactResponse struct {
}

func (*PutArtifactResponse) Descriptor

func (*PutArtifactResponse) Descriptor() ([]byte, []int)

func (*PutArtifactResponse) ProtoMessage

func (*PutArtifactResponse) ProtoMessage()

func (*PutArtifactResponse) Reset

func (m *PutArtifactResponse) Reset()

func (*PutArtifactResponse) String

func (m *PutArtifactResponse) String() string

type RunJobRequest

type RunJobRequest struct {
	// (required) The ID provided by an earlier call to prepare. Runs the job. All prerequisite tasks
	// must have been completed.
	PreparationId string `protobuf:"bytes,1,opt,name=preparation_id,json=preparationId" json:"preparation_id,omitempty"`
	// (optional) If any artifacts have been staged for this job, contains the staging_token returned
	// from the CommitManifestResponse.
	StagingToken string `protobuf:"bytes,2,opt,name=staging_token,json=stagingToken" json:"staging_token,omitempty"`
}

Run is a synchronous request that returns a jobId back. Throws error GRPC_STATUS_UNAVAILABLE if server is down Throws error NOT_FOUND if the preparation ID does not exist Throws error UNKNOWN for all other issues

func (*RunJobRequest) Descriptor

func (*RunJobRequest) Descriptor() ([]byte, []int)

func (*RunJobRequest) GetPreparationId

func (m *RunJobRequest) GetPreparationId() string

func (*RunJobRequest) GetStagingToken

func (m *RunJobRequest) GetStagingToken() string

func (*RunJobRequest) ProtoMessage

func (*RunJobRequest) ProtoMessage()

func (*RunJobRequest) Reset

func (m *RunJobRequest) Reset()

func (*RunJobRequest) String

func (m *RunJobRequest) String() string

type RunJobResponse

type RunJobResponse struct {
	JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId" json:"job_id,omitempty"`
}

func (*RunJobResponse) Descriptor

func (*RunJobResponse) Descriptor() ([]byte, []int)

func (*RunJobResponse) GetJobId

func (m *RunJobResponse) GetJobId() string

func (*RunJobResponse) ProtoMessage

func (*RunJobResponse) ProtoMessage()

func (*RunJobResponse) Reset

func (m *RunJobResponse) Reset()

func (*RunJobResponse) String

func (m *RunJobResponse) String() string

Jump to

Keyboard shortcuts

? : This menu
/ : Search site
f or F : Jump to
y or Y : Canonical URL