fnexecution_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: 8 Imported by: 0

Documentation

Overview

Package fnexecution_v1 is a generated protocol buffer package.

It is generated from these files:

beam_fn_api.proto
beam_provision_api.proto

It has these top-level messages:

Target
RemoteGrpcPort
InstructionRequest
InstructionResponse
RegisterRequest
RegisterResponse
ProcessBundleDescriptor
ProcessBundleRequest
ProcessBundleResponse
ProcessBundleProgressRequest
Metrics
ProcessBundleProgressResponse
ProcessBundleSplitRequest
ElementCountRestriction
ElementCountSkipRestriction
PrimitiveTransformSplit
ProcessBundleSplitResponse
Elements
StateRequest
StateResponse
StateKey
StateGetRequest
StateGetResponse
StateAppendRequest
StateAppendResponse
StateClearRequest
StateClearResponse
LogEntry
LogControl
GetProvisionInfoRequest
GetProvisionInfoResponse
ProvisionInfo
Resources

Index

Constants

This section is empty.

Variables

View Source
var LogEntry_Severity_Enum_name = map[int32]string{
	0: "UNSPECIFIED",
	1: "TRACE",
	2: "DEBUG",
	3: "INFO",
	4: "NOTICE",
	5: "WARN",
	6: "ERROR",
	7: "CRITICAL",
}
View Source
var LogEntry_Severity_Enum_value = map[string]int32{
	"UNSPECIFIED": 0,
	"TRACE":       1,
	"DEBUG":       2,
	"INFO":        3,
	"NOTICE":      4,
	"WARN":        5,
	"ERROR":       6,
	"CRITICAL":    7,
}

Functions

func RegisterBeamFnControlServer

func RegisterBeamFnControlServer(s *grpc.Server, srv BeamFnControlServer)

func RegisterBeamFnDataServer

func RegisterBeamFnDataServer(s *grpc.Server, srv BeamFnDataServer)

func RegisterBeamFnLoggingServer

func RegisterBeamFnLoggingServer(s *grpc.Server, srv BeamFnLoggingServer)

func RegisterBeamFnStateServer

func RegisterBeamFnStateServer(s *grpc.Server, srv BeamFnStateServer)

func RegisterProvisionServiceServer

func RegisterProvisionServiceServer(s *grpc.Server, srv ProvisionServiceServer)

Types

type BeamFnControlClient

type BeamFnControlClient interface {
	// Instructions sent by the runner to the SDK requesting different types
	// of work.
	Control(ctx context.Context, opts ...grpc.CallOption) (BeamFnControl_ControlClient, error)
}

func NewBeamFnControlClient

func NewBeamFnControlClient(cc *grpc.ClientConn) BeamFnControlClient

type BeamFnControlServer

type BeamFnControlServer interface {
	// Instructions sent by the runner to the SDK requesting different types
	// of work.
	Control(BeamFnControl_ControlServer) error
}

type BeamFnControl_ControlClient

type BeamFnControl_ControlClient interface {
	Send(*InstructionResponse) error
	Recv() (*InstructionRequest, error)
	grpc.ClientStream
}

type BeamFnControl_ControlServer

type BeamFnControl_ControlServer interface {
	Send(*InstructionRequest) error
	Recv() (*InstructionResponse, error)
	grpc.ServerStream
}

type BeamFnDataClient

type BeamFnDataClient interface {
	// Used to send data between harnesses.
	Data(ctx context.Context, opts ...grpc.CallOption) (BeamFnData_DataClient, error)
}

func NewBeamFnDataClient

func NewBeamFnDataClient(cc *grpc.ClientConn) BeamFnDataClient

type BeamFnDataServer

type BeamFnDataServer interface {
	// Used to send data between harnesses.
	Data(BeamFnData_DataServer) error
}

type BeamFnData_DataClient

type BeamFnData_DataClient interface {
	Send(*Elements) error
	Recv() (*Elements, error)
	grpc.ClientStream
}

type BeamFnData_DataServer

type BeamFnData_DataServer interface {
	Send(*Elements) error
	Recv() (*Elements, error)
	grpc.ServerStream
}

type BeamFnLoggingClient

type BeamFnLoggingClient interface {
	// Allows for the SDK to emit log entries which the runner can
	// associate with the active job.
	Logging(ctx context.Context, opts ...grpc.CallOption) (BeamFnLogging_LoggingClient, error)
}

func NewBeamFnLoggingClient

func NewBeamFnLoggingClient(cc *grpc.ClientConn) BeamFnLoggingClient

type BeamFnLoggingServer

type BeamFnLoggingServer interface {
	// Allows for the SDK to emit log entries which the runner can
	// associate with the active job.
	Logging(BeamFnLogging_LoggingServer) error
}

type BeamFnLogging_LoggingClient

type BeamFnLogging_LoggingClient interface {
	Send(*LogEntry_List) error
	Recv() (*LogControl, error)
	grpc.ClientStream
}

type BeamFnLogging_LoggingServer

type BeamFnLogging_LoggingServer interface {
	Send(*LogControl) error
	Recv() (*LogEntry_List, error)
	grpc.ServerStream
}

type BeamFnStateClient

type BeamFnStateClient interface {
	// Used to get/append/clear state stored by the runner on behalf of the SDK.
	State(ctx context.Context, opts ...grpc.CallOption) (BeamFnState_StateClient, error)
}

func NewBeamFnStateClient

func NewBeamFnStateClient(cc *grpc.ClientConn) BeamFnStateClient

type BeamFnStateServer

type BeamFnStateServer interface {
	// Used to get/append/clear state stored by the runner on behalf of the SDK.
	State(BeamFnState_StateServer) error
}

type BeamFnState_StateClient

type BeamFnState_StateClient interface {
	Send(*StateRequest) error
	Recv() (*StateResponse, error)
	grpc.ClientStream
}

type BeamFnState_StateServer

type BeamFnState_StateServer interface {
	Send(*StateResponse) error
	Recv() (*StateRequest, error)
	grpc.ServerStream
}

type ElementCountRestriction

type ElementCountRestriction struct {
	// A restriction representing the number of elements that should be processed.
	// Effectively the range [0, count]
	Count int64 `protobuf:"varint,1,opt,name=count" json:"count,omitempty"`
}

urn:org.apache.beam:restriction:element-count:1.0

func (*ElementCountRestriction) Descriptor

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

func (*ElementCountRestriction) GetCount

func (m *ElementCountRestriction) GetCount() int64

func (*ElementCountRestriction) ProtoMessage

func (*ElementCountRestriction) ProtoMessage()

func (*ElementCountRestriction) Reset

func (m *ElementCountRestriction) Reset()

func (*ElementCountRestriction) String

func (m *ElementCountRestriction) String() string

type ElementCountSkipRestriction

type ElementCountSkipRestriction struct {
	// A restriction representing the number of elements that should be skipped.
	// Effectively the range (count, infinity]
	Count int64 `protobuf:"varint,1,opt,name=count" json:"count,omitempty"`
}

urn:org.apache.beam:restriction:element-count-skip:1.0

func (*ElementCountSkipRestriction) Descriptor

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

func (*ElementCountSkipRestriction) GetCount

func (m *ElementCountSkipRestriction) GetCount() int64

func (*ElementCountSkipRestriction) ProtoMessage

func (*ElementCountSkipRestriction) ProtoMessage()

func (*ElementCountSkipRestriction) Reset

func (m *ElementCountSkipRestriction) Reset()

func (*ElementCountSkipRestriction) String

func (m *ElementCountSkipRestriction) String() string

type Elements

type Elements struct {
	// (Required) A list containing parts of logical byte streams.
	Data []*Elements_Data `protobuf:"bytes,1,rep,name=data" json:"data,omitempty"`
}

Messages used to represent logical byte streams. Stable

func (*Elements) Descriptor

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

func (*Elements) GetData

func (m *Elements) GetData() []*Elements_Data

func (*Elements) ProtoMessage

func (*Elements) ProtoMessage()

func (*Elements) Reset

func (m *Elements) Reset()

func (*Elements) String

func (m *Elements) String() string

type Elements_Data

type Elements_Data struct {
	// (Required) A reference to an active instruction request with the given
	// instruction id.
	InstructionReference string `protobuf:"bytes,1,opt,name=instruction_reference,json=instructionReference" json:"instruction_reference,omitempty"`
	// (Required) A definition representing a consumer or producer of this data.
	// If received by a harness, this represents the consumer within that
	// harness that should consume these bytes. If sent by a harness, this
	// represents the producer of these bytes.
	//
	// Note that a single element may span multiple Data messages.
	//
	// Note that a sending/receiving pair should share the same target
	// identifier.
	Target *Target `protobuf:"bytes,2,opt,name=target" json:"target,omitempty"`
	// (Optional) Represents a part of a logical byte stream. Elements within
	// the logical byte stream are encoded in the nested context and
	// concatenated together.
	//
	// An empty data block represents the end of stream for the given
	// instruction and target.
	Data []byte `protobuf:"bytes,3,opt,name=data,proto3" json:"data,omitempty"`
}

Represents multiple encoded elements in nested context for a given named instruction and target.

func (*Elements_Data) Descriptor

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

func (*Elements_Data) GetData

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

func (*Elements_Data) GetInstructionReference

func (m *Elements_Data) GetInstructionReference() string

func (*Elements_Data) GetTarget

func (m *Elements_Data) GetTarget() *Target

func (*Elements_Data) ProtoMessage

func (*Elements_Data) ProtoMessage()

func (*Elements_Data) Reset

func (m *Elements_Data) Reset()

func (*Elements_Data) String

func (m *Elements_Data) String() string

type GetProvisionInfoRequest

type GetProvisionInfoRequest struct {
}

A request to get the provision info of a SDK harness worker instance.

func (*GetProvisionInfoRequest) Descriptor

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

func (*GetProvisionInfoRequest) ProtoMessage

func (*GetProvisionInfoRequest) ProtoMessage()

func (*GetProvisionInfoRequest) Reset

func (m *GetProvisionInfoRequest) Reset()

func (*GetProvisionInfoRequest) String

func (m *GetProvisionInfoRequest) String() string

type GetProvisionInfoResponse

type GetProvisionInfoResponse struct {
	Info *ProvisionInfo `protobuf:"bytes,1,opt,name=info" json:"info,omitempty"`
}

A response containing the provision info of a SDK harness worker instance.

func (*GetProvisionInfoResponse) Descriptor

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

func (*GetProvisionInfoResponse) GetInfo

func (*GetProvisionInfoResponse) ProtoMessage

func (*GetProvisionInfoResponse) ProtoMessage()

func (*GetProvisionInfoResponse) Reset

func (m *GetProvisionInfoResponse) Reset()

func (*GetProvisionInfoResponse) String

func (m *GetProvisionInfoResponse) String() string

type InstructionRequest

type InstructionRequest struct {
	// (Required) An unique identifier provided by the runner which represents
	// this requests execution. The InstructionResponse MUST have the matching id.
	InstructionId string `protobuf:"bytes,1,opt,name=instruction_id,json=instructionId" json:"instruction_id,omitempty"`
	// (Required) A request that the SDK Harness needs to interpret.
	//
	// Types that are valid to be assigned to Request:
	//	*InstructionRequest_Register
	//	*InstructionRequest_ProcessBundle
	//	*InstructionRequest_ProcessBundleProgress
	//	*InstructionRequest_ProcessBundleSplit
	Request isInstructionRequest_Request `protobuf_oneof:"request"`
}

A request sent by a runner which the SDK is asked to fulfill. For any unsupported request type, an error should be returned with a matching instruction id. Stable

func (*InstructionRequest) Descriptor

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

func (*InstructionRequest) GetInstructionId

func (m *InstructionRequest) GetInstructionId() string

func (*InstructionRequest) GetProcessBundle

func (m *InstructionRequest) GetProcessBundle() *ProcessBundleRequest

func (*InstructionRequest) GetProcessBundleProgress

func (m *InstructionRequest) GetProcessBundleProgress() *ProcessBundleProgressRequest

func (*InstructionRequest) GetProcessBundleSplit

func (m *InstructionRequest) GetProcessBundleSplit() *ProcessBundleSplitRequest

func (*InstructionRequest) GetRegister

func (m *InstructionRequest) GetRegister() *RegisterRequest

func (*InstructionRequest) GetRequest

func (m *InstructionRequest) GetRequest() isInstructionRequest_Request

func (*InstructionRequest) ProtoMessage

func (*InstructionRequest) ProtoMessage()

func (*InstructionRequest) Reset

func (m *InstructionRequest) Reset()

func (*InstructionRequest) String

func (m *InstructionRequest) String() string

func (*InstructionRequest) XXX_OneofFuncs

func (*InstructionRequest) 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 InstructionRequest_ProcessBundle

type InstructionRequest_ProcessBundle struct {
	ProcessBundle *ProcessBundleRequest `protobuf:"bytes,1001,opt,name=process_bundle,json=processBundle,oneof"`
}

type InstructionRequest_ProcessBundleProgress

type InstructionRequest_ProcessBundleProgress struct {
	ProcessBundleProgress *ProcessBundleProgressRequest `protobuf:"bytes,1002,opt,name=process_bundle_progress,json=processBundleProgress,oneof"`
}

type InstructionRequest_ProcessBundleSplit

type InstructionRequest_ProcessBundleSplit struct {
	ProcessBundleSplit *ProcessBundleSplitRequest `protobuf:"bytes,1003,opt,name=process_bundle_split,json=processBundleSplit,oneof"`
}

type InstructionRequest_Register

type InstructionRequest_Register struct {
	Register *RegisterRequest `protobuf:"bytes,1000,opt,name=register,oneof"`
}

type InstructionResponse

type InstructionResponse struct {
	// (Required) A reference provided by the runner which represents a requests
	// execution. The InstructionResponse MUST have the matching id when
	// responding to the runner.
	InstructionId string `protobuf:"bytes,1,opt,name=instruction_id,json=instructionId" json:"instruction_id,omitempty"`
	// If this is specified, then this instruction has failed.
	// A human readable string representing the reason as to why processing has
	// failed.
	Error string `protobuf:"bytes,2,opt,name=error" json:"error,omitempty"`
	// If the instruction did not fail, it is required to return an equivalent
	// response type depending on the request this matches.
	//
	// Types that are valid to be assigned to Response:
	//	*InstructionResponse_Register
	//	*InstructionResponse_ProcessBundle
	//	*InstructionResponse_ProcessBundleProgress
	//	*InstructionResponse_ProcessBundleSplit
	Response isInstructionResponse_Response `protobuf_oneof:"response"`
}

The response for an associated request the SDK had been asked to fulfill. Stable

func (*InstructionResponse) Descriptor

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

func (*InstructionResponse) GetError

func (m *InstructionResponse) GetError() string

func (*InstructionResponse) GetInstructionId

func (m *InstructionResponse) GetInstructionId() string

func (*InstructionResponse) GetProcessBundle

func (m *InstructionResponse) GetProcessBundle() *ProcessBundleResponse

func (*InstructionResponse) GetProcessBundleProgress

func (m *InstructionResponse) GetProcessBundleProgress() *ProcessBundleProgressResponse

func (*InstructionResponse) GetProcessBundleSplit

func (m *InstructionResponse) GetProcessBundleSplit() *ProcessBundleSplitResponse

func (*InstructionResponse) GetRegister

func (m *InstructionResponse) GetRegister() *RegisterResponse

func (*InstructionResponse) GetResponse

func (m *InstructionResponse) GetResponse() isInstructionResponse_Response

func (*InstructionResponse) ProtoMessage

func (*InstructionResponse) ProtoMessage()

func (*InstructionResponse) Reset

func (m *InstructionResponse) Reset()

func (*InstructionResponse) String

func (m *InstructionResponse) String() string

func (*InstructionResponse) XXX_OneofFuncs

func (*InstructionResponse) 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 InstructionResponse_ProcessBundle

type InstructionResponse_ProcessBundle struct {
	ProcessBundle *ProcessBundleResponse `protobuf:"bytes,1001,opt,name=process_bundle,json=processBundle,oneof"`
}

type InstructionResponse_ProcessBundleProgress

type InstructionResponse_ProcessBundleProgress struct {
	ProcessBundleProgress *ProcessBundleProgressResponse `protobuf:"bytes,1002,opt,name=process_bundle_progress,json=processBundleProgress,oneof"`
}

type InstructionResponse_ProcessBundleSplit

type InstructionResponse_ProcessBundleSplit struct {
	ProcessBundleSplit *ProcessBundleSplitResponse `protobuf:"bytes,1003,opt,name=process_bundle_split,json=processBundleSplit,oneof"`
}

type InstructionResponse_Register

type InstructionResponse_Register struct {
	Register *RegisterResponse `protobuf:"bytes,1000,opt,name=register,oneof"`
}

type LogControl

type LogControl struct {
}

func (*LogControl) Descriptor

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

func (*LogControl) ProtoMessage

func (*LogControl) ProtoMessage()

func (*LogControl) Reset

func (m *LogControl) Reset()

func (*LogControl) String

func (m *LogControl) String() string

type LogEntry

type LogEntry struct {
	// (Required) The severity of the log statement.
	Severity LogEntry_Severity_Enum `` /* 129-byte string literal not displayed */
	// (Required) The time at which this log statement occurred.
	Timestamp *google_protobuf1.Timestamp `protobuf:"bytes,2,opt,name=timestamp" json:"timestamp,omitempty"`
	// (Required) A human readable message.
	Message string `protobuf:"bytes,3,opt,name=message" json:"message,omitempty"`
	// (Optional) An optional trace of the functions involved. For example, in
	// Java this can include multiple causes and multiple suppressed exceptions.
	Trace string `protobuf:"bytes,4,opt,name=trace" json:"trace,omitempty"`
	// (Optional) A reference to the instruction this log statement is associated
	// with.
	InstructionReference string `protobuf:"bytes,5,opt,name=instruction_reference,json=instructionReference" json:"instruction_reference,omitempty"`
	// (Optional) A reference to the primitive transform this log statement is
	// associated with.
	PrimitiveTransformReference string `` /* 137-byte string literal not displayed */
	// (Optional) Human-readable name of the function or method being invoked,
	// with optional context such as the class or package name. The format can
	// vary by language. For example:
	//   qual.if.ied.Class.method (Java)
	//   dir/package.func (Go)
	//   module.function (Python)
	//   file.cc:382 (C++)
	LogLocation string `protobuf:"bytes,7,opt,name=log_location,json=logLocation" json:"log_location,omitempty"`
	// (Optional) The name of the thread this log statement is associated with.
	Thread string `protobuf:"bytes,8,opt,name=thread" json:"thread,omitempty"`
}

A log entry

func (*LogEntry) Descriptor

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

func (*LogEntry) GetInstructionReference

func (m *LogEntry) GetInstructionReference() string

func (*LogEntry) GetLogLocation

func (m *LogEntry) GetLogLocation() string

func (*LogEntry) GetMessage

func (m *LogEntry) GetMessage() string

func (*LogEntry) GetPrimitiveTransformReference

func (m *LogEntry) GetPrimitiveTransformReference() string

func (*LogEntry) GetSeverity

func (m *LogEntry) GetSeverity() LogEntry_Severity_Enum

func (*LogEntry) GetThread

func (m *LogEntry) GetThread() string

func (*LogEntry) GetTimestamp

func (m *LogEntry) GetTimestamp() *google_protobuf1.Timestamp

func (*LogEntry) GetTrace

func (m *LogEntry) GetTrace() string

func (*LogEntry) ProtoMessage

func (*LogEntry) ProtoMessage()

func (*LogEntry) Reset

func (m *LogEntry) Reset()

func (*LogEntry) String

func (m *LogEntry) String() string

type LogEntry_List

type LogEntry_List struct {
	// (Required) One or or more log messages.
	LogEntries []*LogEntry `protobuf:"bytes,1,rep,name=log_entries,json=logEntries" json:"log_entries,omitempty"`
}

A list of log entries, enables buffering and batching of multiple log messages using the logging API.

func (*LogEntry_List) Descriptor

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

func (*LogEntry_List) GetLogEntries

func (m *LogEntry_List) GetLogEntries() []*LogEntry

func (*LogEntry_List) ProtoMessage

func (*LogEntry_List) ProtoMessage()

func (*LogEntry_List) Reset

func (m *LogEntry_List) Reset()

func (*LogEntry_List) String

func (m *LogEntry_List) String() string

type LogEntry_Severity

type LogEntry_Severity struct {
}

The severity of the event described in a log entry, expressed as one of the severity levels listed below. For your reference, the levels are assigned the listed numeric values. The effect of using numeric values other than those listed is undefined.

If you are writing log entries, you should map other severity encodings to one of these standard levels. For example, you might map all of Java's FINE, FINER, and FINEST levels to `Severity.DEBUG`.

This list is intentionally not comprehensive; the intent is to provide a common set of "good enough" severity levels so that logging front ends can provide filtering and searching across log types. Users of the API are free not to use all severity levels in their log messages.

func (*LogEntry_Severity) Descriptor

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

func (*LogEntry_Severity) ProtoMessage

func (*LogEntry_Severity) ProtoMessage()

func (*LogEntry_Severity) Reset

func (m *LogEntry_Severity) Reset()

func (*LogEntry_Severity) String

func (m *LogEntry_Severity) String() string

type LogEntry_Severity_Enum

type LogEntry_Severity_Enum int32
const (
	// Unspecified level information. Will be logged at the TRACE level.
	LogEntry_Severity_UNSPECIFIED LogEntry_Severity_Enum = 0
	// Trace level information.
	LogEntry_Severity_TRACE LogEntry_Severity_Enum = 1
	// Debugging information.
	LogEntry_Severity_DEBUG LogEntry_Severity_Enum = 2
	// Normal events.
	LogEntry_Severity_INFO LogEntry_Severity_Enum = 3
	// Normal but significant events, such as start up, shut down, or
	// configuration.
	LogEntry_Severity_NOTICE LogEntry_Severity_Enum = 4
	// Warning events might cause problems.
	LogEntry_Severity_WARN LogEntry_Severity_Enum = 5
	// Error events are likely to cause problems.
	LogEntry_Severity_ERROR LogEntry_Severity_Enum = 6
	// Critical events cause severe problems or brief outages and may
	// indicate that a person must take action.
	LogEntry_Severity_CRITICAL LogEntry_Severity_Enum = 7
)

func (LogEntry_Severity_Enum) EnumDescriptor

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

func (LogEntry_Severity_Enum) String

func (x LogEntry_Severity_Enum) String() string

type Metrics

type Metrics struct {
	Ptransforms map[string]*Metrics_PTransform `` /* 142-byte string literal not displayed */
	User        []*Metrics_User                `protobuf:"bytes,2,rep,name=user" json:"user,omitempty"`
}

func (*Metrics) Descriptor

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

func (*Metrics) GetPtransforms

func (m *Metrics) GetPtransforms() map[string]*Metrics_PTransform

func (*Metrics) GetUser

func (m *Metrics) GetUser() []*Metrics_User

func (*Metrics) ProtoMessage

func (*Metrics) ProtoMessage()

func (*Metrics) Reset

func (m *Metrics) Reset()

func (*Metrics) String

func (m *Metrics) String() string

type Metrics_PTransform

type Metrics_PTransform struct {
	// (Required): Metrics for processed elements.
	ProcessedElements *Metrics_PTransform_ProcessedElements `protobuf:"bytes,1,opt,name=processed_elements,json=processedElements" json:"processed_elements,omitempty"`
	// (Required): Metrics for active elements.
	ActiveElements *Metrics_PTransform_ActiveElements `protobuf:"bytes,2,opt,name=active_elements,json=activeElements" json:"active_elements,omitempty"`
	// (Optional): Map from local output name to its watermark.
	// The watermarks reported are tentative, to get a better sense of progress
	// while processing a bundle but before it is committed. At bundle commit
	// time, a Runner needs to also take into account the timers set to compute
	// the actual watermarks.
	Watermarks map[string]int64 `` /* 141-byte string literal not displayed */
}

PTransform level metrics. These metrics are split into processed and active element groups for progress reporting purposes. This allows a Runner to see what is measured, what is estimated and what can be extrapolated to be able to accurately estimate the backlog of remaining work.

func (*Metrics_PTransform) Descriptor

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

func (*Metrics_PTransform) GetActiveElements

func (m *Metrics_PTransform) GetActiveElements() *Metrics_PTransform_ActiveElements

func (*Metrics_PTransform) GetProcessedElements

func (m *Metrics_PTransform) GetProcessedElements() *Metrics_PTransform_ProcessedElements

func (*Metrics_PTransform) GetWatermarks

func (m *Metrics_PTransform) GetWatermarks() map[string]int64

func (*Metrics_PTransform) ProtoMessage

func (*Metrics_PTransform) ProtoMessage()

func (*Metrics_PTransform) Reset

func (m *Metrics_PTransform) Reset()

func (*Metrics_PTransform) String

func (m *Metrics_PTransform) String() string

type Metrics_PTransform_ActiveElements

type Metrics_PTransform_ActiveElements struct {
	// (Required)
	Measured *Metrics_PTransform_Measured `protobuf:"bytes,1,opt,name=measured" json:"measured,omitempty"`
	// (Optional) Sum of estimated fraction of known work remaining for all
	// active elements, as reported by this transform.
	// If not reported, a Runner could extrapolate this from the processed
	// elements.
	// TODO: Handle the case when known work is infinite.
	FractionRemaining float64 `protobuf:"fixed64,2,opt,name=fraction_remaining,json=fractionRemaining" json:"fraction_remaining,omitempty"`
	// (Optional) Map from local output name to sum of estimated number
	// of elements remaining for this output from all active elements,
	// as reported by this transform.
	// If not reported, a Runner could extrapolate this from the processed
	// elements.
	OutputElementsRemaining map[string]int64 `` /* 200-byte string literal not displayed */
}

Metrics for active elements. An element is considered active if the SDK has started but not finished processing it yet.

func (*Metrics_PTransform_ActiveElements) Descriptor

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

func (*Metrics_PTransform_ActiveElements) GetFractionRemaining

func (m *Metrics_PTransform_ActiveElements) GetFractionRemaining() float64

func (*Metrics_PTransform_ActiveElements) GetMeasured

func (*Metrics_PTransform_ActiveElements) GetOutputElementsRemaining

func (m *Metrics_PTransform_ActiveElements) GetOutputElementsRemaining() map[string]int64

func (*Metrics_PTransform_ActiveElements) ProtoMessage

func (*Metrics_PTransform_ActiveElements) ProtoMessage()

func (*Metrics_PTransform_ActiveElements) Reset

func (*Metrics_PTransform_ActiveElements) String

type Metrics_PTransform_Measured

type Metrics_PTransform_Measured struct {
	// (Optional) Map from local input name to number of elements processed
	// from this input.
	// If unset, assumed to be the sum of the outputs of all producers to
	// this transform (for ProcessedElements) and 0 (for ActiveElements).
	InputElementCounts map[string]int64 `` /* 185-byte string literal not displayed */
	// (Required) Map from local output name to number of elements produced
	// for this output.
	OutputElementCounts map[string]int64 `` /* 188-byte string literal not displayed */
	// (Optional) The total time spent so far in processing the elements in
	// this group, in seconds.
	TotalTimeSpent float64 `protobuf:"fixed64,3,opt,name=total_time_spent,json=totalTimeSpent" json:"total_time_spent,omitempty"`
}

Metrics that are measured for processed and active element groups.

func (*Metrics_PTransform_Measured) Descriptor

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

func (*Metrics_PTransform_Measured) GetInputElementCounts

func (m *Metrics_PTransform_Measured) GetInputElementCounts() map[string]int64

func (*Metrics_PTransform_Measured) GetOutputElementCounts

func (m *Metrics_PTransform_Measured) GetOutputElementCounts() map[string]int64

func (*Metrics_PTransform_Measured) GetTotalTimeSpent

func (m *Metrics_PTransform_Measured) GetTotalTimeSpent() float64

func (*Metrics_PTransform_Measured) ProtoMessage

func (*Metrics_PTransform_Measured) ProtoMessage()

func (*Metrics_PTransform_Measured) Reset

func (m *Metrics_PTransform_Measured) Reset()

func (*Metrics_PTransform_Measured) String

func (m *Metrics_PTransform_Measured) String() string

type Metrics_PTransform_ProcessedElements

type Metrics_PTransform_ProcessedElements struct {
	// (Required)
	Measured *Metrics_PTransform_Measured `protobuf:"bytes,1,opt,name=measured" json:"measured,omitempty"`
}

Metrics for fully processed elements.

func (*Metrics_PTransform_ProcessedElements) Descriptor

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

func (*Metrics_PTransform_ProcessedElements) GetMeasured

func (*Metrics_PTransform_ProcessedElements) ProtoMessage

func (*Metrics_PTransform_ProcessedElements) ProtoMessage()

func (*Metrics_PTransform_ProcessedElements) Reset

func (*Metrics_PTransform_ProcessedElements) String

type Metrics_User

type Metrics_User struct {
	// (Required) The identifier for this metric.
	Key *Metrics_User_MetricKey `protobuf:"bytes,1,opt,name=key" json:"key,omitempty"`
	// (Required) The data for this metric.
	//
	// Types that are valid to be assigned to Data:
	//	*Metrics_User_CounterData_
	//	*Metrics_User_DistributionData_
	Data isMetrics_User_Data `protobuf_oneof:"data"`
}

User defined metrics

func (*Metrics_User) Descriptor

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

func (*Metrics_User) GetCounterData

func (m *Metrics_User) GetCounterData() *Metrics_User_CounterData

func (*Metrics_User) GetData

func (m *Metrics_User) GetData() isMetrics_User_Data

func (*Metrics_User) GetDistributionData

func (m *Metrics_User) GetDistributionData() *Metrics_User_DistributionData

func (*Metrics_User) GetKey

func (m *Metrics_User) GetKey() *Metrics_User_MetricKey

func (*Metrics_User) ProtoMessage

func (*Metrics_User) ProtoMessage()

func (*Metrics_User) Reset

func (m *Metrics_User) Reset()

func (*Metrics_User) String

func (m *Metrics_User) String() string

func (*Metrics_User) XXX_OneofFuncs

func (*Metrics_User) 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 Metrics_User_CounterData

type Metrics_User_CounterData struct {
	Value int64 `protobuf:"varint,1,opt,name=value" json:"value,omitempty"`
}

Data associated with a counter metric.

func (*Metrics_User_CounterData) Descriptor

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

func (*Metrics_User_CounterData) GetValue

func (m *Metrics_User_CounterData) GetValue() int64

func (*Metrics_User_CounterData) ProtoMessage

func (*Metrics_User_CounterData) ProtoMessage()

func (*Metrics_User_CounterData) Reset

func (m *Metrics_User_CounterData) Reset()

func (*Metrics_User_CounterData) String

func (m *Metrics_User_CounterData) String() string

type Metrics_User_CounterData_

type Metrics_User_CounterData_ struct {
	CounterData *Metrics_User_CounterData `protobuf:"bytes,1001,opt,name=counter_data,json=counterData,oneof"`
}

type Metrics_User_DistributionData

type Metrics_User_DistributionData struct {
	Count int64 `protobuf:"varint,1,opt,name=count" json:"count,omitempty"`
	Sum   int64 `protobuf:"varint,2,opt,name=sum" json:"sum,omitempty"`
	Min   int64 `protobuf:"varint,3,opt,name=min" json:"min,omitempty"`
	Max   int64 `protobuf:"varint,4,opt,name=max" json:"max,omitempty"`
}

Data associated with a distribution metric.

func (*Metrics_User_DistributionData) Descriptor

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

func (*Metrics_User_DistributionData) GetCount

func (m *Metrics_User_DistributionData) GetCount() int64

func (*Metrics_User_DistributionData) GetMax

func (*Metrics_User_DistributionData) GetMin

func (*Metrics_User_DistributionData) GetSum

func (*Metrics_User_DistributionData) ProtoMessage

func (*Metrics_User_DistributionData) ProtoMessage()

func (*Metrics_User_DistributionData) Reset

func (m *Metrics_User_DistributionData) Reset()

func (*Metrics_User_DistributionData) String

type Metrics_User_DistributionData_

type Metrics_User_DistributionData_ struct {
	DistributionData *Metrics_User_DistributionData `protobuf:"bytes,1002,opt,name=distribution_data,json=distributionData,oneof"`
}

type Metrics_User_MetricKey

type Metrics_User_MetricKey struct {
	// The step, if any, this metric is associated with.
	Step string `protobuf:"bytes,1,opt,name=step" json:"step,omitempty"`
	// (Required): The namespace of this metric.
	Namespace string `protobuf:"bytes,2,opt,name=namespace" json:"namespace,omitempty"`
	// (Required): The name of this metric.
	Name string `protobuf:"bytes,3,opt,name=name" json:"name,omitempty"`
}

A key for identifying a metric at the most granular level.

func (*Metrics_User_MetricKey) Descriptor

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

func (*Metrics_User_MetricKey) GetName

func (m *Metrics_User_MetricKey) GetName() string

func (*Metrics_User_MetricKey) GetNamespace

func (m *Metrics_User_MetricKey) GetNamespace() string

func (*Metrics_User_MetricKey) GetStep

func (m *Metrics_User_MetricKey) GetStep() string

func (*Metrics_User_MetricKey) ProtoMessage

func (*Metrics_User_MetricKey) ProtoMessage()

func (*Metrics_User_MetricKey) Reset

func (m *Metrics_User_MetricKey) Reset()

func (*Metrics_User_MetricKey) String

func (m *Metrics_User_MetricKey) String() string

type PrimitiveTransformSplit

type PrimitiveTransformSplit struct {
	// (Required) A reference to a primitive transform with the given id that
	// is part of the active process bundle request with the given instruction
	// id.
	PrimitiveTransformReference string `` /* 137-byte string literal not displayed */
	// (Required) A function specification describing the restriction
	// that has been completed by the primitive transform.
	//
	// For example, a remote GRPC source will have a specific urn and data
	// block containing an ElementCountRestriction.
	CompletedRestriction *org_apache_beam_model_pipeline_v1.FunctionSpec `protobuf:"bytes,2,opt,name=completed_restriction,json=completedRestriction" json:"completed_restriction,omitempty"`
	// (Required) A function specification describing the restriction
	// representing the remainder of work for the primitive transform.
	//
	// FOr example, a remote GRPC source will have a specific urn and data
	// block contain an ElemntCountSkipRestriction.
	RemainingRestriction *org_apache_beam_model_pipeline_v1.FunctionSpec `protobuf:"bytes,3,opt,name=remaining_restriction,json=remainingRestriction" json:"remaining_restriction,omitempty"`
}

Each primitive transform that is splittable is defined by a restriction it is currently processing. During splitting, that currently active restriction (R_initial) is split into 2 components:

  • a restriction (R_done) representing all elements that will be fully processed
  • a restriction (R_todo) representing all elements that will not be fully processed

where:

R_initial = R_done ⋃ R_todo

func (*PrimitiveTransformSplit) Descriptor

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

func (*PrimitiveTransformSplit) GetCompletedRestriction

func (*PrimitiveTransformSplit) GetPrimitiveTransformReference

func (m *PrimitiveTransformSplit) GetPrimitiveTransformReference() string

func (*PrimitiveTransformSplit) GetRemainingRestriction

func (*PrimitiveTransformSplit) ProtoMessage

func (*PrimitiveTransformSplit) ProtoMessage()

func (*PrimitiveTransformSplit) Reset

func (m *PrimitiveTransformSplit) Reset()

func (*PrimitiveTransformSplit) String

func (m *PrimitiveTransformSplit) String() string

type ProcessBundleDescriptor

type ProcessBundleDescriptor struct {
	// (Required) A pipeline level unique id which can be used as a reference to
	// refer to this.
	Id string `protobuf:"bytes,1,opt,name=id" json:"id,omitempty"`
	// (Required) A map from pipeline-scoped id to PTransform.
	Transforms map[string]*org_apache_beam_model_pipeline_v1.PTransform `` /* 140-byte string literal not displayed */
	// (Required) A map from pipeline-scoped id to PCollection.
	Pcollections map[string]*org_apache_beam_model_pipeline_v1.PCollection `` /* 144-byte string literal not displayed */
	// (Required) A map from pipeline-scoped id to WindowingStrategy.
	WindowingStrategies map[string]*org_apache_beam_model_pipeline_v1.WindowingStrategy `` /* 185-byte string literal not displayed */
	// (Required) A map from pipeline-scoped id to Coder.
	Coders map[string]*org_apache_beam_model_pipeline_v1.Coder `` /* 132-byte string literal not displayed */
	// (Required) A map from pipeline-scoped id to Environment.
	Environments map[string]*org_apache_beam_model_pipeline_v1.Environment `` /* 144-byte string literal not displayed */
	// A descriptor describing the end point to use for State API
	// calls. Required if the Runner intends to send remote references over the
	// data plane or if any of the transforms rely on user state or side inputs.
	StateApiServiceDescriptor *org_apache_beam_model_pipeline_v11.ApiServiceDescriptor `` /* 133-byte string literal not displayed */
}

Definitions that should be used to construct the bundle processing graph.

func (*ProcessBundleDescriptor) Descriptor

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

func (*ProcessBundleDescriptor) GetCoders

func (*ProcessBundleDescriptor) GetEnvironments

func (*ProcessBundleDescriptor) GetId

func (m *ProcessBundleDescriptor) GetId() string

func (*ProcessBundleDescriptor) GetPcollections

func (*ProcessBundleDescriptor) GetStateApiServiceDescriptor

func (*ProcessBundleDescriptor) GetTransforms

func (*ProcessBundleDescriptor) GetWindowingStrategies

func (*ProcessBundleDescriptor) ProtoMessage

func (*ProcessBundleDescriptor) ProtoMessage()

func (*ProcessBundleDescriptor) Reset

func (m *ProcessBundleDescriptor) Reset()

func (*ProcessBundleDescriptor) String

func (m *ProcessBundleDescriptor) String() string

type ProcessBundleProgressRequest

type ProcessBundleProgressRequest struct {
	// (Required) A reference to an active process bundle request with the given
	// instruction id.
	InstructionReference string `protobuf:"bytes,1,opt,name=instruction_reference,json=instructionReference" json:"instruction_reference,omitempty"`
}

A request to report progress information for a given bundle. This is an optional request to be handled and is used to support advanced SDK features such as SplittableDoFn, user level metrics etc.

func (*ProcessBundleProgressRequest) Descriptor

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

func (*ProcessBundleProgressRequest) GetInstructionReference

func (m *ProcessBundleProgressRequest) GetInstructionReference() string

func (*ProcessBundleProgressRequest) ProtoMessage

func (*ProcessBundleProgressRequest) ProtoMessage()

func (*ProcessBundleProgressRequest) Reset

func (m *ProcessBundleProgressRequest) Reset()

func (*ProcessBundleProgressRequest) String

type ProcessBundleProgressResponse

type ProcessBundleProgressResponse struct {
	// (Required)
	Metrics *Metrics `protobuf:"bytes,1,opt,name=metrics" json:"metrics,omitempty"`
}

func (*ProcessBundleProgressResponse) Descriptor

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

func (*ProcessBundleProgressResponse) GetMetrics

func (m *ProcessBundleProgressResponse) GetMetrics() *Metrics

func (*ProcessBundleProgressResponse) ProtoMessage

func (*ProcessBundleProgressResponse) ProtoMessage()

func (*ProcessBundleProgressResponse) Reset

func (m *ProcessBundleProgressResponse) Reset()

func (*ProcessBundleProgressResponse) String

type ProcessBundleRequest

type ProcessBundleRequest struct {
	// (Required) A reference to the process bundle descriptor that must be
	// instantiated and executed by the SDK harness.
	ProcessBundleDescriptorReference string `` /* 154-byte string literal not displayed */
	// (Optional) A list of cache tokens that can be used by an SDK to reuse
	// cached data returned by the State API across multiple bundles.
	CacheTokens [][]byte `protobuf:"bytes,2,rep,name=cache_tokens,json=cacheTokens,proto3" json:"cache_tokens,omitempty"`
}

A request to process a given bundle. Stable

func (*ProcessBundleRequest) Descriptor

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

func (*ProcessBundleRequest) GetCacheTokens

func (m *ProcessBundleRequest) GetCacheTokens() [][]byte

func (*ProcessBundleRequest) GetProcessBundleDescriptorReference

func (m *ProcessBundleRequest) GetProcessBundleDescriptorReference() string

func (*ProcessBundleRequest) ProtoMessage

func (*ProcessBundleRequest) ProtoMessage()

func (*ProcessBundleRequest) Reset

func (m *ProcessBundleRequest) Reset()

func (*ProcessBundleRequest) String

func (m *ProcessBundleRequest) String() string

type ProcessBundleResponse

type ProcessBundleResponse struct {
	// (Optional) If metrics reporting is supported by the SDK, this represents
	// the final metrics to record for this bundle.
	Metrics *Metrics `protobuf:"bytes,1,opt,name=metrics" json:"metrics,omitempty"`
}

Stable

func (*ProcessBundleResponse) Descriptor

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

func (*ProcessBundleResponse) GetMetrics

func (m *ProcessBundleResponse) GetMetrics() *Metrics

func (*ProcessBundleResponse) ProtoMessage

func (*ProcessBundleResponse) ProtoMessage()

func (*ProcessBundleResponse) Reset

func (m *ProcessBundleResponse) Reset()

func (*ProcessBundleResponse) String

func (m *ProcessBundleResponse) String() string

type ProcessBundleSplitRequest

type ProcessBundleSplitRequest struct {
	// (Required) A reference to an active process bundle request with the given
	// instruction id.
	InstructionReference string `protobuf:"bytes,1,opt,name=instruction_reference,json=instructionReference" json:"instruction_reference,omitempty"`
	// (Required) The fraction of work (when compared to the known amount of work)
	// the process bundle request should try to split at.
	Fraction float64 `protobuf:"fixed64,2,opt,name=fraction" json:"fraction,omitempty"`
}

func (*ProcessBundleSplitRequest) Descriptor

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

func (*ProcessBundleSplitRequest) GetFraction

func (m *ProcessBundleSplitRequest) GetFraction() float64

func (*ProcessBundleSplitRequest) GetInstructionReference

func (m *ProcessBundleSplitRequest) GetInstructionReference() string

func (*ProcessBundleSplitRequest) ProtoMessage

func (*ProcessBundleSplitRequest) ProtoMessage()

func (*ProcessBundleSplitRequest) Reset

func (m *ProcessBundleSplitRequest) Reset()

func (*ProcessBundleSplitRequest) String

func (m *ProcessBundleSplitRequest) String() string

type ProcessBundleSplitResponse

type ProcessBundleSplitResponse struct {
	// If primitive transform B and C are siblings and descendants of A and A, B,
	// and C report a split. Then B and C's restrictions are relative to A's.
	//   R = A_done
	//     ⋃ (A_boundary ⋂ B_done)
	//     ⋃ (A_boundary ⋂ B_todo)
	//     ⋃ (A_boundary ⋂ B_todo)
	//     ⋃ (A_boundary ⋂ C_todo)
	//     ⋃ A_todo
	// If there is no descendant of B or C also reporting a split, than
	//   B_boundary = ∅ and C_boundary = ∅
	//
	// This restriction is processed and completed by the currently active process
	// bundle request:
	//   A_done ⋃ (A_boundary ⋂ B_done)
	//          ⋃ (A_boundary ⋂ C_done)
	// and these restrictions will be processed by future process bundle requests:
	//   A_boundary ⋂ B_todo (passed to SDF B directly)
	//   A_boundary ⋂ C_todo (passed to SDF C directly)
	//   A_todo (passed to SDF A directly)
	//
	// Note that descendants splits should only be reported if it is inexpensive
	// to compute the boundary restriction intersected with descendants splits.
	// Also note, that the boundary restriction may represent a set of elements
	// produced by a parent primitive transform which can not be split at each
	// element or that there are intermediate unsplittable primitive transforms
	// between an ancestor splittable function and a descendant splittable
	// function which may have more than one output per element. Finally note
	// that the descendant splits should only be reported if the split
	// information is relatively compact.
	Splits []*PrimitiveTransformSplit `protobuf:"bytes,1,rep,name=splits" json:"splits,omitempty"`
}

func (*ProcessBundleSplitResponse) Descriptor

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

func (*ProcessBundleSplitResponse) GetSplits

func (*ProcessBundleSplitResponse) ProtoMessage

func (*ProcessBundleSplitResponse) ProtoMessage()

func (*ProcessBundleSplitResponse) Reset

func (m *ProcessBundleSplitResponse) Reset()

func (*ProcessBundleSplitResponse) String

func (m *ProcessBundleSplitResponse) String() string

type ProvisionInfo

type ProvisionInfo struct {
	// (required) The job ID.
	JobId string `protobuf:"bytes,1,opt,name=job_id,json=jobId" json:"job_id,omitempty"`
	// (required) The job name.
	JobName string `protobuf:"bytes,2,opt,name=job_name,json=jobName" json:"job_name,omitempty"`
	// (required) The worker ID.
	WorkerId string `protobuf:"bytes,5,opt,name=worker_id,json=workerId" json:"worker_id,omitempty"`
	// (required) Pipeline options. For non-template jobs, the options are
	// identical to what is passed to job submission.
	PipelineOptions *google_protobuf2.Struct `protobuf:"bytes,3,opt,name=pipeline_options,json=pipelineOptions" json:"pipeline_options,omitempty"`
	// (optional) Resource limits that the SDK harness worker should respect.
	// Runners may -- but are not required to -- enforce any limits provided.
	ResourceLimits *Resources `protobuf:"bytes,4,opt,name=resource_limits,json=resourceLimits" json:"resource_limits,omitempty"`
}

Runtime provisioning information for a SDK harness worker instance, such as pipeline options, resource constraints and other job metadata

func (*ProvisionInfo) Descriptor

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

func (*ProvisionInfo) GetJobId

func (m *ProvisionInfo) GetJobId() string

func (*ProvisionInfo) GetJobName

func (m *ProvisionInfo) GetJobName() string

func (*ProvisionInfo) GetPipelineOptions

func (m *ProvisionInfo) GetPipelineOptions() *google_protobuf2.Struct

func (*ProvisionInfo) GetResourceLimits

func (m *ProvisionInfo) GetResourceLimits() *Resources

func (*ProvisionInfo) GetWorkerId

func (m *ProvisionInfo) GetWorkerId() string

func (*ProvisionInfo) ProtoMessage

func (*ProvisionInfo) ProtoMessage()

func (*ProvisionInfo) Reset

func (m *ProvisionInfo) Reset()

func (*ProvisionInfo) String

func (m *ProvisionInfo) String() string

type ProvisionServiceClient

type ProvisionServiceClient interface {
	// Get provision information for the SDK harness worker instance.
	GetProvisionInfo(ctx context.Context, in *GetProvisionInfoRequest, opts ...grpc.CallOption) (*GetProvisionInfoResponse, error)
}

func NewProvisionServiceClient

func NewProvisionServiceClient(cc *grpc.ClientConn) ProvisionServiceClient

type ProvisionServiceServer

type ProvisionServiceServer interface {
	// Get provision information for the SDK harness worker instance.
	GetProvisionInfo(context.Context, *GetProvisionInfoRequest) (*GetProvisionInfoResponse, error)
}

type RegisterRequest

type RegisterRequest struct {
	// (Optional) The set of descriptors used to process bundles.
	ProcessBundleDescriptor []*ProcessBundleDescriptor `protobuf:"bytes,1,rep,name=process_bundle_descriptor,json=processBundleDescriptor" json:"process_bundle_descriptor,omitempty"`
}

A list of objects which can be referred to by the runner in future requests. Stable

func (*RegisterRequest) Descriptor

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

func (*RegisterRequest) GetProcessBundleDescriptor

func (m *RegisterRequest) GetProcessBundleDescriptor() []*ProcessBundleDescriptor

func (*RegisterRequest) ProtoMessage

func (*RegisterRequest) ProtoMessage()

func (*RegisterRequest) Reset

func (m *RegisterRequest) Reset()

func (*RegisterRequest) String

func (m *RegisterRequest) String() string

type RegisterResponse

type RegisterResponse struct {
}

Stable

func (*RegisterResponse) Descriptor

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

func (*RegisterResponse) ProtoMessage

func (*RegisterResponse) ProtoMessage()

func (*RegisterResponse) Reset

func (m *RegisterResponse) Reset()

func (*RegisterResponse) String

func (m *RegisterResponse) String() string

type RemoteGrpcPort

type RemoteGrpcPort struct {
	// (Required) An API descriptor which describes where to
	// connect to including any authentication that is required.
	ApiServiceDescriptor *org_apache_beam_model_pipeline_v11.ApiServiceDescriptor `protobuf:"bytes,1,opt,name=api_service_descriptor,json=apiServiceDescriptor" json:"api_service_descriptor,omitempty"`
}

A descriptor for connecting to a remote port using the Beam Fn Data API. Allows for communication between two environments (for example between the runner and the SDK). Stable

func (*RemoteGrpcPort) Descriptor

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

func (*RemoteGrpcPort) GetApiServiceDescriptor

func (*RemoteGrpcPort) ProtoMessage

func (*RemoteGrpcPort) ProtoMessage()

func (*RemoteGrpcPort) Reset

func (m *RemoteGrpcPort) Reset()

func (*RemoteGrpcPort) String

func (m *RemoteGrpcPort) String() string

type Resources

type Resources struct {
	// (optional) Memory usage limits. SDKs can use this value to configure
	// internal buffer sizes and language specific sizes.
	Memory *Resources_Memory `protobuf:"bytes,1,opt,name=memory" json:"memory,omitempty"`
	// (optional) CPU usage limits.
	Cpu *Resources_Cpu `protobuf:"bytes,2,opt,name=cpu" json:"cpu,omitempty"`
	// (optional) Disk size limits for the semi-persistent location.
	SemiPersistentDisk *Resources_Disk `protobuf:"bytes,3,opt,name=semi_persistent_disk,json=semiPersistentDisk" json:"semi_persistent_disk,omitempty"`
}

Resources specify limits for local resources, such memory and cpu. It is used to inform SDK harnesses of their allocated footprint.

func (*Resources) Descriptor

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

func (*Resources) GetCpu

func (m *Resources) GetCpu() *Resources_Cpu

func (*Resources) GetMemory

func (m *Resources) GetMemory() *Resources_Memory

func (*Resources) GetSemiPersistentDisk

func (m *Resources) GetSemiPersistentDisk() *Resources_Disk

func (*Resources) ProtoMessage

func (*Resources) ProtoMessage()

func (*Resources) Reset

func (m *Resources) Reset()

func (*Resources) String

func (m *Resources) String() string

type Resources_Cpu

type Resources_Cpu struct {
	// (optional) Shares of a cpu to use. Fractional values, such as "0.2"
	// or "2.5", are fine. Any value <= 0 means unspecified.
	Shares float32 `protobuf:"fixed32,1,opt,name=shares" json:"shares,omitempty"`
}

CPU limits.

func (*Resources_Cpu) Descriptor

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

func (*Resources_Cpu) GetShares

func (m *Resources_Cpu) GetShares() float32

func (*Resources_Cpu) ProtoMessage

func (*Resources_Cpu) ProtoMessage()

func (*Resources_Cpu) Reset

func (m *Resources_Cpu) Reset()

func (*Resources_Cpu) String

func (m *Resources_Cpu) String() string

type Resources_Disk

type Resources_Disk struct {
	// (optional) Hard limit in bytes. A zero value means unspecified.
	Size uint64 `protobuf:"varint,1,opt,name=size" json:"size,omitempty"`
}

Disk limits.

func (*Resources_Disk) Descriptor

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

func (*Resources_Disk) GetSize

func (m *Resources_Disk) GetSize() uint64

func (*Resources_Disk) ProtoMessage

func (*Resources_Disk) ProtoMessage()

func (*Resources_Disk) Reset

func (m *Resources_Disk) Reset()

func (*Resources_Disk) String

func (m *Resources_Disk) String() string

type Resources_Memory

type Resources_Memory struct {
	// (optional) Hard limit in bytes. A zero value means unspecified.
	Size uint64 `protobuf:"varint,1,opt,name=size" json:"size,omitempty"`
}

Memory limits.

func (*Resources_Memory) Descriptor

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

func (*Resources_Memory) GetSize

func (m *Resources_Memory) GetSize() uint64

func (*Resources_Memory) ProtoMessage

func (*Resources_Memory) ProtoMessage()

func (*Resources_Memory) Reset

func (m *Resources_Memory) Reset()

func (*Resources_Memory) String

func (m *Resources_Memory) String() string

type StateAppendRequest

type StateAppendRequest struct {
	// Represents a part of a logical byte stream. Elements within
	// the logical byte stream are encoded in the nested context and
	// multiple append requests are concatenated together.
	Data []byte `protobuf:"bytes,1,opt,name=data,proto3" json:"data,omitempty"`
}

A request to append state.

func (*StateAppendRequest) Descriptor

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

func (*StateAppendRequest) GetData

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

func (*StateAppendRequest) ProtoMessage

func (*StateAppendRequest) ProtoMessage()

func (*StateAppendRequest) Reset

func (m *StateAppendRequest) Reset()

func (*StateAppendRequest) String

func (m *StateAppendRequest) String() string

type StateAppendResponse

type StateAppendResponse struct {
}

A response to append state.

func (*StateAppendResponse) Descriptor

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

func (*StateAppendResponse) ProtoMessage

func (*StateAppendResponse) ProtoMessage()

func (*StateAppendResponse) Reset

func (m *StateAppendResponse) Reset()

func (*StateAppendResponse) String

func (m *StateAppendResponse) String() string

type StateClearRequest

type StateClearRequest struct {
}

A request to clear state.

func (*StateClearRequest) Descriptor

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

func (*StateClearRequest) ProtoMessage

func (*StateClearRequest) ProtoMessage()

func (*StateClearRequest) Reset

func (m *StateClearRequest) Reset()

func (*StateClearRequest) String

func (m *StateClearRequest) String() string

type StateClearResponse

type StateClearResponse struct {
}

A response to clear state.

func (*StateClearResponse) Descriptor

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

func (*StateClearResponse) ProtoMessage

func (*StateClearResponse) ProtoMessage()

func (*StateClearResponse) Reset

func (m *StateClearResponse) Reset()

func (*StateClearResponse) String

func (m *StateClearResponse) String() string

type StateGetRequest

type StateGetRequest struct {
	// (Optional) If specified, signals to the runner that the response
	// should resume from the following continuation token.
	//
	// If unspecified, signals to the runner that the response should start
	// from the beginning of the logical continuable stream.
	ContinuationToken []byte `protobuf:"bytes,1,opt,name=continuation_token,json=continuationToken,proto3" json:"continuation_token,omitempty"`
}

A request to get state.

func (*StateGetRequest) Descriptor

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

func (*StateGetRequest) GetContinuationToken

func (m *StateGetRequest) GetContinuationToken() []byte

func (*StateGetRequest) ProtoMessage

func (*StateGetRequest) ProtoMessage()

func (*StateGetRequest) Reset

func (m *StateGetRequest) Reset()

func (*StateGetRequest) String

func (m *StateGetRequest) String() string

type StateGetResponse

type StateGetResponse struct {
	// (Optional) If specified, represents a token which can be used with the
	// state API to get the next chunk of this logical byte stream. The end of
	// the logical byte stream is signalled by this field being unset.
	ContinuationToken []byte `protobuf:"bytes,1,opt,name=continuation_token,json=continuationToken,proto3" json:"continuation_token,omitempty"`
	// Represents a part of a logical byte stream. Elements within
	// the logical byte stream are encoded in the nested context and
	// concatenated together.
	Data []byte `protobuf:"bytes,2,opt,name=data,proto3" json:"data,omitempty"`
}

A response to get state representing a logical byte stream which can be continued using the state API.

func (*StateGetResponse) Descriptor

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

func (*StateGetResponse) GetContinuationToken

func (m *StateGetResponse) GetContinuationToken() []byte

func (*StateGetResponse) GetData

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

func (*StateGetResponse) ProtoMessage

func (*StateGetResponse) ProtoMessage()

func (*StateGetResponse) Reset

func (m *StateGetResponse) Reset()

func (*StateGetResponse) String

func (m *StateGetResponse) String() string

type StateKey

type StateKey struct {
	// (Required) One of the following state keys must be set.
	//
	// Types that are valid to be assigned to Type:
	//	*StateKey_Runner_
	//	*StateKey_MultimapSideInput_
	//	*StateKey_BagUserState_
	Type isStateKey_Type `protobuf_oneof:"type"`
}

func (*StateKey) Descriptor

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

func (*StateKey) GetBagUserState

func (m *StateKey) GetBagUserState() *StateKey_BagUserState

func (*StateKey) GetMultimapSideInput

func (m *StateKey) GetMultimapSideInput() *StateKey_MultimapSideInput

func (*StateKey) GetRunner

func (m *StateKey) GetRunner() *StateKey_Runner

func (*StateKey) GetType

func (m *StateKey) GetType() isStateKey_Type

func (*StateKey) ProtoMessage

func (*StateKey) ProtoMessage()

func (*StateKey) Reset

func (m *StateKey) Reset()

func (*StateKey) String

func (m *StateKey) String() string

func (*StateKey) XXX_OneofFuncs

func (*StateKey) 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 StateKey_BagUserState

type StateKey_BagUserState struct {
	// (Required) The id of the PTransform containing user state.
	PtransformId string `protobuf:"bytes,1,opt,name=ptransform_id,json=ptransformId" json:"ptransform_id,omitempty"`
	// (Required) The id of the user state.
	UserStateId string `protobuf:"bytes,2,opt,name=user_state_id,json=userStateId" json:"user_state_id,omitempty"`
	// (Required) The window encoded in a nested context.
	Window []byte `protobuf:"bytes,3,opt,name=window,proto3" json:"window,omitempty"`
	// (Required) The key of the currently executing element encoded in a
	// nested context.
	Key []byte `protobuf:"bytes,4,opt,name=key,proto3" json:"key,omitempty"`
}

func (*StateKey_BagUserState) Descriptor

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

func (*StateKey_BagUserState) GetKey

func (m *StateKey_BagUserState) GetKey() []byte

func (*StateKey_BagUserState) GetPtransformId

func (m *StateKey_BagUserState) GetPtransformId() string

func (*StateKey_BagUserState) GetUserStateId

func (m *StateKey_BagUserState) GetUserStateId() string

func (*StateKey_BagUserState) GetWindow

func (m *StateKey_BagUserState) GetWindow() []byte

func (*StateKey_BagUserState) ProtoMessage

func (*StateKey_BagUserState) ProtoMessage()

func (*StateKey_BagUserState) Reset

func (m *StateKey_BagUserState) Reset()

func (*StateKey_BagUserState) String

func (m *StateKey_BagUserState) String() string

type StateKey_BagUserState_

type StateKey_BagUserState_ struct {
	BagUserState *StateKey_BagUserState `protobuf:"bytes,3,opt,name=bag_user_state,json=bagUserState,oneof"`
}

type StateKey_MultimapSideInput

type StateKey_MultimapSideInput struct {
	// (Required) The id of the PTransform containing a side input.
	PtransformId string `protobuf:"bytes,1,opt,name=ptransform_id,json=ptransformId" json:"ptransform_id,omitempty"`
	// (Required) The id of the side input.
	SideInputId string `protobuf:"bytes,2,opt,name=side_input_id,json=sideInputId" json:"side_input_id,omitempty"`
	// (Required) The window (after mapping the currently executing elements
	// window into the side input windows domain) encoded in a nested context.
	Window []byte `protobuf:"bytes,3,opt,name=window,proto3" json:"window,omitempty"`
	// (Required) The key encoded in a nested context.
	Key []byte `protobuf:"bytes,4,opt,name=key,proto3" json:"key,omitempty"`
}

func (*StateKey_MultimapSideInput) Descriptor

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

func (*StateKey_MultimapSideInput) GetKey

func (m *StateKey_MultimapSideInput) GetKey() []byte

func (*StateKey_MultimapSideInput) GetPtransformId

func (m *StateKey_MultimapSideInput) GetPtransformId() string

func (*StateKey_MultimapSideInput) GetSideInputId

func (m *StateKey_MultimapSideInput) GetSideInputId() string

func (*StateKey_MultimapSideInput) GetWindow

func (m *StateKey_MultimapSideInput) GetWindow() []byte

func (*StateKey_MultimapSideInput) ProtoMessage

func (*StateKey_MultimapSideInput) ProtoMessage()

func (*StateKey_MultimapSideInput) Reset

func (m *StateKey_MultimapSideInput) Reset()

func (*StateKey_MultimapSideInput) String

func (m *StateKey_MultimapSideInput) String() string

type StateKey_MultimapSideInput_

type StateKey_MultimapSideInput_ struct {
	MultimapSideInput *StateKey_MultimapSideInput `protobuf:"bytes,2,opt,name=multimap_side_input,json=multimapSideInput,oneof"`
}

type StateKey_Runner

type StateKey_Runner struct {
	// (Required) Opaque information supplied by the runner. Used to support
	// remote references.
	Key []byte `protobuf:"bytes,1,opt,name=key,proto3" json:"key,omitempty"`
}

func (*StateKey_Runner) Descriptor

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

func (*StateKey_Runner) GetKey

func (m *StateKey_Runner) GetKey() []byte

func (*StateKey_Runner) ProtoMessage

func (*StateKey_Runner) ProtoMessage()

func (*StateKey_Runner) Reset

func (m *StateKey_Runner) Reset()

func (*StateKey_Runner) String

func (m *StateKey_Runner) String() string

type StateKey_Runner_

type StateKey_Runner_ struct {
	Runner *StateKey_Runner `protobuf:"bytes,1,opt,name=runner,oneof"`
}

type StateRequest

type StateRequest struct {
	// (Required) An unique identifier provided by the SDK which represents this
	// requests execution. The StateResponse corresponding with this request
	// will have the matching id.
	Id string `protobuf:"bytes,1,opt,name=id" json:"id,omitempty"`
	// (Required) The associated instruction id of the work that is currently
	// being processed. This allows for the runner to associate any modifications
	// to state to be committed with the appropriate work execution.
	InstructionReference string `protobuf:"bytes,2,opt,name=instruction_reference,json=instructionReference" json:"instruction_reference,omitempty"`
	// (Required) The state key this request is for.
	StateKey *StateKey `protobuf:"bytes,3,opt,name=state_key,json=stateKey" json:"state_key,omitempty"`
	// (Required) The action to take on this request.
	//
	// Types that are valid to be assigned to Request:
	//	*StateRequest_Get
	//	*StateRequest_Append
	//	*StateRequest_Clear
	Request isStateRequest_Request `protobuf_oneof:"request"`
}

func (*StateRequest) Descriptor

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

func (*StateRequest) GetAppend

func (m *StateRequest) GetAppend() *StateAppendRequest

func (*StateRequest) GetClear

func (m *StateRequest) GetClear() *StateClearRequest

func (*StateRequest) GetGet

func (m *StateRequest) GetGet() *StateGetRequest

func (*StateRequest) GetId

func (m *StateRequest) GetId() string

func (*StateRequest) GetInstructionReference

func (m *StateRequest) GetInstructionReference() string

func (*StateRequest) GetRequest

func (m *StateRequest) GetRequest() isStateRequest_Request

func (*StateRequest) GetStateKey

func (m *StateRequest) GetStateKey() *StateKey

func (*StateRequest) ProtoMessage

func (*StateRequest) ProtoMessage()

func (*StateRequest) Reset

func (m *StateRequest) Reset()

func (*StateRequest) String

func (m *StateRequest) String() string

func (*StateRequest) XXX_OneofFuncs

func (*StateRequest) 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 StateRequest_Append

type StateRequest_Append struct {
	Append *StateAppendRequest `protobuf:"bytes,1001,opt,name=append,oneof"`
}

type StateRequest_Clear

type StateRequest_Clear struct {
	Clear *StateClearRequest `protobuf:"bytes,1002,opt,name=clear,oneof"`
}

type StateRequest_Get

type StateRequest_Get struct {
	Get *StateGetRequest `protobuf:"bytes,1000,opt,name=get,oneof"`
}

type StateResponse

type StateResponse struct {
	// (Required) A reference provided by the SDK which represents a requests
	// execution. The StateResponse must have the matching id when responding
	// to the SDK.
	Id string `protobuf:"bytes,1,opt,name=id" json:"id,omitempty"`
	// (Optional) If this is specified, then the state request has failed.
	// A human readable string representing the reason as to why the request
	// failed.
	Error string `protobuf:"bytes,2,opt,name=error" json:"error,omitempty"`
	// (Optional) If this is specified, then the result of this state request
	// can be cached using the supplied token.
	CacheToken []byte `protobuf:"bytes,3,opt,name=cache_token,json=cacheToken,proto3" json:"cache_token,omitempty"`
	// A corresponding response matching the request will be populated.
	//
	// Types that are valid to be assigned to Response:
	//	*StateResponse_Get
	//	*StateResponse_Append
	//	*StateResponse_Clear
	Response isStateResponse_Response `protobuf_oneof:"response"`
}

func (*StateResponse) Descriptor

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

func (*StateResponse) GetAppend

func (m *StateResponse) GetAppend() *StateAppendResponse

func (*StateResponse) GetCacheToken

func (m *StateResponse) GetCacheToken() []byte

func (*StateResponse) GetClear

func (m *StateResponse) GetClear() *StateClearResponse

func (*StateResponse) GetError

func (m *StateResponse) GetError() string

func (*StateResponse) GetGet

func (m *StateResponse) GetGet() *StateGetResponse

func (*StateResponse) GetId

func (m *StateResponse) GetId() string

func (*StateResponse) GetResponse

func (m *StateResponse) GetResponse() isStateResponse_Response

func (*StateResponse) ProtoMessage

func (*StateResponse) ProtoMessage()

func (*StateResponse) Reset

func (m *StateResponse) Reset()

func (*StateResponse) String

func (m *StateResponse) String() string

func (*StateResponse) XXX_OneofFuncs

func (*StateResponse) 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 StateResponse_Append

type StateResponse_Append struct {
	Append *StateAppendResponse `protobuf:"bytes,1001,opt,name=append,oneof"`
}

type StateResponse_Clear

type StateResponse_Clear struct {
	Clear *StateClearResponse `protobuf:"bytes,1002,opt,name=clear,oneof"`
}

type StateResponse_Get

type StateResponse_Get struct {
	Get *StateGetResponse `protobuf:"bytes,1000,opt,name=get,oneof"`
}

type Target

type Target struct {
	// (Required) The id of the PrimitiveTransform which is the target.
	PrimitiveTransformReference string `` /* 137-byte string literal not displayed */
	// (Required) The local name of an input or output defined on the primitive
	// transform.
	Name string `protobuf:"bytes,2,opt,name=name" json:"name,omitempty"`
}

A representation of an input or output definition on a primitive transform. Stable

func (*Target) Descriptor

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

func (*Target) GetName

func (m *Target) GetName() string

func (*Target) GetPrimitiveTransformReference

func (m *Target) GetPrimitiveTransformReference() string

func (*Target) ProtoMessage

func (*Target) ProtoMessage()

func (*Target) Reset

func (m *Target) Reset()

func (*Target) String

func (m *Target) String() string

type Target_List

type Target_List struct {
	Target []*Target `protobuf:"bytes,1,rep,name=target" json:"target,omitempty"`
}

A repeated list of target definitions.

func (*Target_List) Descriptor

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

func (*Target_List) GetTarget

func (m *Target_List) GetTarget() []*Target

func (*Target_List) ProtoMessage

func (*Target_List) ProtoMessage()

func (*Target_List) Reset

func (m *Target_List) Reset()

func (*Target_List) String

func (m *Target_List) String() string

Jump to

Keyboard shortcuts

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