@DefaultAnnotation(value=org.checkerframework.checker.nullness.qual.NonNull.class)
PTransform
s for transforming data in a pipeline.See: Description
Interface | Description |
---|---|
Combine.AccumulatingCombineFn.Accumulator<InputT,AccumT,OutputT> |
The type of mutable accumulator values used by this
AccumulatingCombineFn . |
CombineFnBase.GlobalCombineFn<InputT,AccumT,OutputT> |
For internal use only; no backwards-compatibility guarantees.
|
CombineWithContext.RequiresContextInternal |
An internal interface for signaling that a
GloballyCombineFn or a PerKeyCombineFn needs to access CombineWithContext.Context . |
Contextful.Fn<InputT,OutputT> |
A function from an input to an output that may additionally access
Contextful.Fn.Context when
computing the result. |
DoFn.BundleFinalizer |
A parameter that is accessible during
@StartBundle , @ProcessElement and @FinishBundle that allows the caller
to register a callback that will be invoked after the bundle has been successfully completed
and the runner has commit the output. |
DoFn.BundleFinalizer.Callback |
An instance of a function that will be invoked after bundle finalization.
|
DoFn.MultiOutputReceiver |
Receives tagged output for a multi-output function.
|
DoFn.OutputReceiver<T> |
Receives values of the given type.
|
ExternalTransformBuilder<ConfigT,InputT extends PInput,OutputT extends POutput> |
An interface for building a transform from an externally provided configuration.
|
Materialization<T> |
For internal use only; no backwards-compatibility guarantees.
|
Materializations.IterableView<V> |
Represents the
PrimitiveViewT supplied to the ViewFn when it declares to use
the iterable materialization . |
Materializations.MultimapView<K,V> |
Represents the
PrimitiveViewT supplied to the ViewFn when it declares to use
the multimap materialization . |
Partition.PartitionFn<T> |
A function object that chooses an output partition for an element.
|
Partition.PartitionWithSideInputsFn<T> |
A function object that chooses an output partition for an element.
|
ProcessFunction<InputT,OutputT> |
A function that computes an output value of type
OutputT from an input value of type
InputT and is Serializable . |
SerializableBiConsumer<FirstInputT,SecondInputT> |
A union of the
BiConsumer and Serializable interfaces. |
SerializableBiFunction<FirstInputT,SecondInputT,OutputT> |
A union of the
BiFunction and Serializable interfaces. |
SerializableComparator<T> |
A
Comparator that is also Serializable . |
SerializableFunction<InputT,OutputT> |
A function that computes an output value of type
OutputT from an input value of type
InputT , is Serializable , and does not allow checked exceptions to be declared. |
Watch.Growth.TerminationCondition<InputT,StateT> |
A strategy for determining whether it is time to stop polling the current input regardless of
whether its output is complete or not.
|
Class | Description |
---|---|
ApproximateQuantiles |
PTransform s for getting an idea of a PCollection 's data distribution using
approximate N -tiles (e.g. |
ApproximateQuantiles.ApproximateQuantilesCombineFn<T,ComparatorT extends java.util.Comparator<T> & java.io.Serializable> |
The
ApproximateQuantilesCombineFn combiner gives an idea of the distribution of a
collection of values using approximate N -tiles. |
ApproximateUnique | Deprecated
Consider using
ApproximateCountDistinct in the zetasketch extension
module, which makes use of the HllCount implementation. |
ApproximateUnique.ApproximateUniqueCombineFn<T> |
CombineFn that computes an estimate of the number of distinct values that were
combined. |
ApproximateUnique.ApproximateUniqueCombineFn.LargestUnique |
A heap utility class to efficiently track the largest added elements.
|
ApproximateUnique.Globally<T> |
PTransform for estimating the number of distinct elements in a PCollection . |
ApproximateUnique.PerKey<K,V> |
PTransform for estimating the number of distinct values associated with each key in a
PCollection of KV s. |
Combine |
PTransform s for combining PCollection elements globally and per-key. |
Combine.AccumulatingCombineFn<InputT,AccumT extends Combine.AccumulatingCombineFn.Accumulator<InputT,AccumT,OutputT>,OutputT> |
A
CombineFn that uses a subclass of Combine.AccumulatingCombineFn.Accumulator as its
accumulator type. |
Combine.BinaryCombineDoubleFn |
An abstract subclass of
Combine.CombineFn for implementing combiners that are more easily and
efficiently expressed as binary operations on double s. |
Combine.BinaryCombineFn<V> |
An abstract subclass of
Combine.CombineFn for implementing combiners that are more easily
expressed as binary operations. |
Combine.BinaryCombineIntegerFn |
An abstract subclass of
Combine.CombineFn for implementing combiners that are more easily and
efficiently expressed as binary operations on int s |
Combine.BinaryCombineLongFn |
An abstract subclass of
Combine.CombineFn for implementing combiners that are more easily and
efficiently expressed as binary operations on long s. |
Combine.CombineFn<InputT,AccumT,OutputT> |
A
CombineFn<InputT, AccumT, OutputT> specifies how to combine a collection of input
values of type InputT into a single output value of type OutputT . |
Combine.Globally<InputT,OutputT> |
Combine.Globally<InputT, OutputT> takes a PCollection<InputT> and returns a
PCollection<OutputT> whose elements are the result of combining all the elements in
each window of the input PCollection , using a specified CombineFn<InputT, AccumT, OutputT> . |
Combine.GloballyAsSingletonView<InputT,OutputT> |
Combine.GloballyAsSingletonView<InputT, OutputT> takes a PCollection<InputT>
and returns a PCollectionView<OutputT> whose elements are the result of combining all
the elements in each window of the input PCollection , using a specified CombineFn<InputT, AccumT, OutputT> . |
Combine.GroupedValues<K,InputT,OutputT> |
GroupedValues<K, InputT, OutputT> takes a PCollection<KV<K, Iterable<InputT>>> ,
such as the result of GroupByKey , applies a specified CombineFn<InputT, AccumT, OutputT> to each of the input KV<K, Iterable<InputT>>
elements to produce a combined output KV<K, OutputT> element, and returns a PCollection<KV<K, OutputT>> containing all the combined output elements. |
Combine.Holder<V> |
Holds a single value value of type
V which may or may not be present. |
Combine.IterableCombineFn<V> | |
Combine.PerKey<K,InputT,OutputT> |
PerKey<K, InputT, OutputT> takes a PCollection<KV<K, InputT>> , groups it by
key, applies a combining function to the InputT values associated with each key to
produce a combined OutputT value, and returns a PCollection<KV<K, OutputT>>
representing a map from each distinct key of the input PCollection to the corresponding
combined value. |
Combine.PerKeyWithHotKeyFanout<K,InputT,OutputT> |
Like
Combine.PerKey , but sharding the combining of hot keys. |
Combine.SimpleCombineFn<V> | Deprecated |
CombineFnBase |
For internal use only; no backwards-compatibility guarantees.
|
CombineFns |
Static utility methods that create combine function instances.
|
CombineFns.CoCombineResult |
A tuple of outputs produced by a composed combine functions.
|
CombineFns.ComposeCombineFnBuilder |
A builder class to construct a composed
CombineFnBase.GlobalCombineFn . |
CombineFns.ComposedCombineFn<DataT> |
A composed
Combine.CombineFn that applies multiple CombineFns . |
CombineFns.ComposedCombineFnWithContext<DataT> |
A composed
CombineWithContext.CombineFnWithContext that applies multiple CombineFnWithContexts . |
CombineWithContext |
This class contains combine functions that have access to
PipelineOptions and side inputs
through CombineWithContext.Context . |
CombineWithContext.CombineFnWithContext<InputT,AccumT,OutputT> |
A combine function that has access to
PipelineOptions and side inputs through CombineWithContext.Context . |
CombineWithContext.Context |
Information accessible to all methods in
CombineFnWithContext and KeyedCombineFnWithContext . |
Contextful<ClosureT> |
Pair of a bit of user code (a "closure") and the
Requirements needed to run it. |
Contextful.Fn.Context |
An accessor for additional capabilities available in
Contextful.Fn.apply(InputT, org.apache.beam.sdk.transforms.Contextful.Fn.Context) . |
Count |
PTransforms to count the elements in a PCollection . |
Create<T> |
Create<T> takes a collection of elements of type T known when the pipeline is
constructed and returns a PCollection<T> containing the elements. |
Create.OfValueProvider<T> | |
Create.TimestampedValues<T> |
A
PTransform that creates a PCollection whose elements have associated
timestamps. |
Create.Values<T> |
A
PTransform that creates a PCollection from a set of in-memory objects. |
Create.WindowedValues<T> |
A
PTransform that creates a PCollection whose elements have associated
windowing metadata. |
Deduplicate |
A set of
PTransform s which deduplicate input records over a time domain and threshold. |
Deduplicate.KeyedValues<K,V> |
Deduplicates keyed values using the key over a specified time domain and threshold.
|
Deduplicate.Values<T> |
Deduplicates values over a specified time domain and threshold.
|
Deduplicate.WithRepresentativeValues<T,IdT> |
A
PTransform that uses a SerializableFunction to obtain a representative value
for each input element used for deduplication. |
Distinct<T> |
Distinct<T> takes a PCollection<T> and returns a PCollection<T> that has
all distinct elements of the input. |
Distinct.WithRepresentativeValues<T,IdT> |
A
Distinct PTransform that uses a SerializableFunction to obtain a
representative value for each input element. |
DoFn<InputT,OutputT> |
The argument to
ParDo providing the code to use to process elements of the input PCollection . |
DoFn.ProcessContinuation |
When used as a return value of
DoFn.ProcessElement , indicates whether there is more work to
be done for the current element. |
DoFnOutputReceivers |
Common
DoFn.OutputReceiver and DoFn.MultiOutputReceiver classes. |
DoFnSchemaInformation |
Represents information about how a DoFn extracts schemas.
|
DoFnSchemaInformation.Builder |
The builder object.
|
DoFnTester<InputT,OutputT> | Deprecated
Use
TestPipeline with the DirectRunner . |
Filter<T> |
PTransform s for filtering from a PCollection the elements satisfying a predicate,
or satisfying an inequality with a given value based on the elements' natural ordering. |
FlatMapElements<InputT,OutputT> |
PTransform s for mapping a simple function that returns iterables over the elements of a
PCollection and merging the results. |
FlatMapElements.FlatMapWithFailures<InputT,OutputT,FailureT> |
A
PTransform that adds exception handling to FlatMapElements . |
Flatten |
Flatten<T> takes multiple PCollection<T> s bundled into a PCollectionList<T> and returns a single PCollection<T> containing all the elements in
all the input PCollection s. |
Flatten.Iterables<T> |
FlattenIterables<T> takes a PCollection<Iterable<T>> and returns a PCollection<T> that contains all the elements from each iterable. |
Flatten.PCollections<T> |
A
PTransform that flattens a PCollectionList into a PCollection
containing all the elements of all the PCollection s in its input. |
GroupByKey<K,V> |
GroupByKey<K, V> takes a PCollection<KV<K, V>> , groups the values by key and
windows, and returns a PCollection<KV<K, Iterable<V>>> representing a map from each
distinct key and window of the input PCollection to an Iterable over all the
values associated with that key in the input per window. |
GroupIntoBatches<K,InputT> |
A
PTransform that batches inputs to a desired batch size. |
GroupIntoBatches.BatchingParams<InputT> |
Wrapper class for batching parameters supplied by users.
|
Impulse |
For internal use only; no backwards-compatibility guarantees.
|
InferableFunction<InputT,OutputT> |
A
ProcessFunction which is not a functional interface. |
JsonToRow | |
JsonToRow.JsonToRowWithErrFn | |
JsonToRow.JsonToRowWithErrFn.Builder | |
JsonToRow.JsonToRowWithErrFn.ParseWithError | |
JsonToRow.JsonToRowWithErrFn.ParseWithError.Builder | |
JsonToRow.ParseResult |
The result of a
JsonToRow.withExceptionReporting(Schema) transform. |
JsonToRow.ParseResult.Builder | |
Keys<K> |
Keys<K> takes a PCollection of KV<K, V> s and returns a PCollection<K> of the keys. |
KvSwap<K,V> |
KvSwap<K, V> takes a PCollection<KV<K, V>> and returns a PCollection<KV<V,
K>> , where all the keys and values have been swapped. |
Latest | |
MapElements<InputT,OutputT> |
PTransform s for mapping a simple function over the elements of a PCollection . |
MapElements.MapWithFailures<InputT,OutputT,FailureT> |
A
PTransform that adds exception handling to MapElements . |
MapKeys<K1,K2,V> |
MapKeys maps a SerializableFunction<K1,K2> over keys of a PCollection<KV<K1,V>> and returns a PCollection<KV<K2, V>> . |
MapValues<K,V1,V2> |
MapValues maps a SerializableFunction<V1,V2> over values of a PCollection<KV<K,V1>> and returns a PCollection<KV<K, V2>> . |
Materializations |
For internal use only; no backwards-compatibility guarantees.
|
Max |
PTransform s for computing the maximum of the elements in a PCollection , or the
maximum of the values associated with each key in a PCollection of KV s. |
Mean |
PTransform s for computing the arithmetic mean (a.k.a. |
Min |
PTransform s for computing the minimum of the elements in a PCollection , or the
minimum of the values associated with each key in a PCollection of KV s. |
ParDo |
ParDo is the core element-wise transform in Apache Beam, invoking a user-specified
function on each of the elements of the input PCollection to produce zero or more output
elements, all of which are collected into the output PCollection . |
ParDo.MultiOutput<InputT,OutputT> |
A
PTransform that, when applied to a PCollection<InputT> , invokes a
user-specified DoFn<InputT, OutputT> on all its elements, which can emit elements to
any of the PTransform 's output PCollection s, which are bundled into a result
PCollectionTuple . |
ParDo.SingleOutput<InputT,OutputT> |
A
PTransform that, when applied to a PCollection<InputT> , invokes a
user-specified DoFn<InputT, OutputT> on all its elements, with all its outputs
collected into an output PCollection<OutputT> . |
Partition<T> |
Partition takes a PCollection<T> and a PartitionFn , uses the PartitionFn to split the elements of the input PCollection into N partitions,
and returns a PCollectionList<T> that bundles N PCollection<T> s
containing the split elements. |
PeriodicImpulse |
A
PTransform which produces a sequence of elements at fixed runtime intervals. |
PeriodicSequence |
A
PTransform which generates a sequence of timestamped elements at given runtime
intervals. |
PeriodicSequence.OutputRangeTracker | |
PeriodicSequence.SequenceDefinition | |
PTransform<InputT extends PInput,OutputT extends POutput> | |
Redistribute |
A family of
PTransforms that returns a PCollection equivalent to its
input but functions as an operational hint to a runner that redistributing the data in some way
is likely useful. |
Redistribute.RedistributeArbitrarily<T> |
Noop transform that hints to the runner to try to redistribute the work evenly, or via whatever
clever strategy the runner comes up with.
|
Redistribute.RedistributeByKey<K,V> | |
Redistribute.Registrar |
Registers translators for the Redistribute family of transforms.
|
Regex |
PTransform s to use Regular Expressions to process elements in a PCollection . |
Regex.AllMatches |
Regex.MatchesName<String> takes a PCollection<String> and returns a PCollection<List<String>> representing the value extracted from all the Regex groups of the
input PCollection to the number of times that element occurs in the input. |
Regex.Find |
Regex.Find<String> takes a PCollection<String> and returns a PCollection<String> representing the value extracted from the Regex groups of the input PCollection to the number of times that element occurs in the input. |
Regex.FindAll |
Regex.Find<String> takes a PCollection<String> and returns a PCollection<List<String>> representing the value extracted from the Regex groups of the input
PCollection to the number of times that element occurs in the input. |
Regex.FindKV |
Regex.MatchesKV<KV<String, String>> takes a PCollection<String> and returns a
PCollection<KV<String, String>> representing the key and value extracted from the Regex
groups of the input PCollection to the number of times that element occurs in the
input. |
Regex.FindName |
Regex.Find<String> takes a PCollection<String> and returns a PCollection<String> representing the value extracted from the Regex groups of the input PCollection to the number of times that element occurs in the input. |
Regex.FindNameKV |
Regex.MatchesKV<KV<String, String>> takes a PCollection<String> and returns a
PCollection<KV<String, String>> representing the key and value extracted from the Regex
groups of the input PCollection to the number of times that element occurs in the
input. |
Regex.Matches |
Regex.Matches<String> takes a PCollection<String> and returns a PCollection<String> representing the value extracted from the Regex groups of the input PCollection to the number of times that element occurs in the input. |
Regex.MatchesKV |
Regex.MatchesKV<KV<String, String>> takes a PCollection<String> and returns a
PCollection<KV<String, String>> representing the key and value extracted from the Regex
groups of the input PCollection to the number of times that element occurs in the
input. |
Regex.MatchesName |
Regex.MatchesName<String> takes a PCollection<String> and returns a PCollection<String> representing the value extracted from the Regex groups of the input PCollection to the number of times that element occurs in the input. |
Regex.MatchesNameKV |
Regex.MatchesNameKV<KV<String, String>> takes a PCollection<String> and returns
a PCollection<KV<String, String>> representing the key and value extracted from the
Regex groups of the input PCollection to the number of times that element occurs in the
input. |
Regex.ReplaceAll |
Regex.ReplaceAll<String> takes a PCollection<String> and returns a PCollection<String> with all Strings that matched the Regex being replaced with the
replacement string. |
Regex.ReplaceFirst |
Regex.ReplaceFirst<String> takes a PCollection<String> and returns a PCollection<String> with the first Strings that matched the Regex being replaced with the
replacement string. |
Regex.Split |
Regex.Split<String> takes a PCollection<String> and returns a PCollection<String> with the input string split into individual items in a list. |
Reify |
PTransforms for converting between explicit and implicit form of various Beam
values. |
Requirements |
Describes the run-time requirements of a
Contextful , such as access to side inputs. |
Reshuffle<K,V> |
For internal use only; no backwards compatibility guarantees.
|
Reshuffle.AssignShardFn<T> | |
Reshuffle.ViaRandomKey<T> |
Implementation of
Reshuffle.viaRandomKey() . |
Sample |
PTransform s for taking samples of the elements in a PCollection , or samples of
the values associated with each key in a PCollection of KV s. |
Sample.FixedSizedSampleFn<T> |
CombineFn that computes a fixed-size sample of a collection of values. |
SerializableFunctions |
Useful
SerializableFunction overrides. |
Sets |
The
PTransform s that allow to compute different set functions across PCollection s. |
SimpleFunction<InputT,OutputT> |
A
SerializableFunction which is not a functional interface. |
Sum |
PTransform s for computing the sum of the elements in a PCollection , or the sum of
the values associated with each key in a PCollection of KV s. |
ToJson<T> |
Creates a
PTransform that serializes UTF-8 JSON objects from a Schema -aware
PCollection (i.e. |
Top |
PTransform s for finding the largest (or smallest) set of elements in a PCollection , or the largest (or smallest) set of values associated with each key in a PCollection of KV s. |
Top.Largest<T extends java.lang.Comparable<? super T>> | Deprecated
use
Top.Natural instead |
Top.Natural<T extends java.lang.Comparable<? super T>> |
A
Serializable Comparator that that uses the compared elements' natural
ordering. |
Top.Reversed<T extends java.lang.Comparable<? super T>> |
Serializable Comparator that that uses the reverse of the compared elements'
natural ordering. |
Top.Smallest<T extends java.lang.Comparable<? super T>> | Deprecated
use
Top.Reversed instead |
Top.TopCombineFn<T,ComparatorT extends java.util.Comparator<T> & java.io.Serializable> |
CombineFn for Top transforms that combines a bunch of T s into a single
count -long List<T> , using compareFn to choose the largest T s. |
ToString |
PTransforms for converting a PCollection<?> , PCollection<KV<?,?>> , or PCollection<Iterable<?>> to a PCollection<String> . |
Values<V> |
Values<V> takes a PCollection of KV<K, V> s and returns a PCollection<V> of the values. |
View |
Transforms for creating
PCollectionViews from PCollections (to read them as side inputs). |
View.AsIterable<T> |
For internal use only; no backwards-compatibility guarantees.
|
View.AsList<T> |
For internal use only; no backwards-compatibility guarantees.
|
View.AsMap<K,V> |
For internal use only; no backwards-compatibility guarantees.
|
View.AsMultimap<K,V> |
For internal use only; no backwards-compatibility guarantees.
|
View.AsSingleton<T> |
For internal use only; no backwards-compatibility guarantees.
|
View.CreatePCollectionView<ElemT,ViewT> |
For internal use only; no backwards-compatibility guarantees.
|
View.ToListViewDoFn<T> |
Provides an index to value mapping using a random starting index and also provides an offset
range for each window seen.
|
ViewFn<PrimitiveViewT,ViewT> |
For internal use only; no backwards-compatibility guarantees.
|
Wait |
Delays processing of each window in a
PCollection until signaled. |
Wait.OnSignal<T> |
Implementation of
Wait.on(org.apache.beam.sdk.values.PCollection<?>...) . |
Watch |
Given a "poll function" that produces a potentially growing set of outputs for an input, this
transform simultaneously continuously watches the growth of output sets of all inputs, until a
per-input termination condition is reached.
|
Watch.Growth<InputT,OutputT,KeyT> | |
Watch.Growth.PollFn<InputT,OutputT> |
A function that computes the current set of outputs for the given input, in the form of a
Watch.Growth.PollResult . |
Watch.Growth.PollResult<OutputT> |
The result of a single invocation of a
Watch.Growth.PollFn . |
Watch.WatchGrowthFn<InputT,OutputT,KeyT,TerminationStateT> | |
WithFailures |
A collection of utilities for writing transforms that can handle exceptions raised during
processing of elements.
|
WithFailures.ExceptionAsMapHandler<T> |
A simple handler that extracts information from an exception to a
Map<String, String>
and returns a KV where the key is the input element that failed processing, and the
value is the map of exception attributes. |
WithFailures.ExceptionElement<T> |
The value type passed as input to exception handlers.
|
WithFailures.Result<OutputT extends POutput,FailureElementT> |
An intermediate output type for PTransforms that allows an output collection to live alongside
a collection of elements that failed the transform.
|
WithFailures.ThrowableHandler<T> |
A handler that holds onto the
Throwable that led to the exception, returning it along
with the original value as a KV . |
WithKeys<K,V> |
WithKeys<K, V> takes a PCollection<V> , and either a constant key of type K or a function from V to K , and returns a PCollection<KV<K, V>> , where
each of the values in the input PCollection has been paired with either the constant key
or a key computed from the value. |
WithTimestamps<T> |
A
PTransform for assigning timestamps to all the elements of a PCollection . |
Enum | Description |
---|---|
DoFnTester.CloningBehavior | Deprecated
Use
TestPipeline with the DirectRunner . |
Annotation Type | Description |
---|---|
DoFn.AlwaysFetched |
Annotation for declaring that a state parameter is always fetched.
|
DoFn.BoundedPerElement |
Annotation on a splittable
DoFn
specifying that the DoFn performs a bounded amount of work per input element, so
applying it to a bounded PCollection will produce also a bounded PCollection . |
DoFn.Element |
Parameter annotation for the input element for
DoFn.ProcessElement , DoFn.GetInitialRestriction , DoFn.GetSize , DoFn.SplitRestriction , DoFn.GetInitialWatermarkEstimatorState , DoFn.NewWatermarkEstimator , and DoFn.NewTracker
methods. |
DoFn.FieldAccess |
Annotation for specifying specific fields that are accessed in a Schema PCollection.
|
DoFn.FinishBundle |
Annotation for the method to use to finish processing a batch of elements.
|
DoFn.GetInitialRestriction |
Annotation for the method that maps an element to an initial restriction for a splittable
DoFn . |
DoFn.GetInitialWatermarkEstimatorState |
Annotation for the method that maps an element and restriction to initial watermark estimator
state for a splittable
DoFn . |
DoFn.GetRestrictionCoder |
Annotation for the method that returns the coder to use for the restriction of a splittable
DoFn . |
DoFn.GetSize |
Annotation for the method that returns the corresponding size for an element and restriction
pair.
|
DoFn.GetWatermarkEstimatorStateCoder |
Annotation for the method that returns the coder to use for the watermark estimator state of a
splittable
DoFn . |
DoFn.Key |
Parameter annotation for dereferencing input element key in
KV pair. |
DoFn.NewTracker |
Annotation for the method that creates a new
RestrictionTracker for the restriction of
a splittable DoFn . |
DoFn.NewWatermarkEstimator |
Annotation for the method that creates a new
WatermarkEstimator for the watermark state
of a splittable DoFn . |
DoFn.OnTimer |
Annotation for registering a callback for a timer.
|
DoFn.OnTimerFamily |
Annotation for registering a callback for a timerFamily.
|
DoFn.OnWindowExpiration |
Annotation for the method to use for performing actions on window expiration.
|
DoFn.ProcessElement |
Annotation for the method to use for processing elements.
|
DoFn.RequiresStableInput |
Annotation that may be added to a
DoFn.ProcessElement , DoFn.OnTimer , or DoFn.OnWindowExpiration method to indicate that the runner must ensure that the observable contents
of the input PCollection or mutable state must be stable upon retries. |
DoFn.RequiresTimeSortedInput |
Annotation that may be added to a
DoFn.ProcessElement method to indicate that the runner
must ensure that the observable contents of the input PCollection is sorted by time, in
ascending order. |
DoFn.Restriction |
Parameter annotation for the restriction for
DoFn.GetSize , DoFn.SplitRestriction , DoFn.GetInitialWatermarkEstimatorState , DoFn.NewWatermarkEstimator , and DoFn.NewTracker
methods. |
DoFn.Setup |
Annotation for the method to use to prepare an instance for processing bundles of elements.
|
DoFn.SideInput |
Parameter annotation for the SideInput for a
DoFn.ProcessElement method. |
DoFn.SplitRestriction |
Annotation for the method that splits restriction of a splittable
DoFn into multiple parts to
be processed in parallel. |
DoFn.StartBundle |
Annotation for the method to use to prepare an instance for processing a batch of elements.
|
DoFn.StateId |
Annotation for declaring and dereferencing state cells.
|
DoFn.Teardown |
Annotation for the method to use to clean up this instance before it is discarded.
|
DoFn.TimerFamily |
Parameter annotation for the TimerMap for a
DoFn.ProcessElement method. |
DoFn.TimerId |
Annotation for declaring and dereferencing timers.
|
DoFn.Timestamp |
Parameter annotation for the input element timestamp for
DoFn.ProcessElement , DoFn.GetInitialRestriction , DoFn.GetSize , DoFn.SplitRestriction , DoFn.GetInitialWatermarkEstimatorState , DoFn.NewWatermarkEstimator , and DoFn.NewTracker
methods. |
DoFn.TruncateRestriction |
Annotation for the method that truncates the restriction of a splittable
DoFn into a bounded one. |
DoFn.UnboundedPerElement |
Annotation on a splittable
DoFn
specifying that the DoFn performs an unbounded amount of work per input element, so
applying it to a bounded PCollection will produce an unbounded PCollection . |
DoFn.WatermarkEstimatorState |
Parameter annotation for the watermark estimator state for the
DoFn.NewWatermarkEstimator
method. |
PTransform
s for transforming data in a pipeline.
A PTransform
is an operation that takes an InputT
(some subtype of PInput
) and produces an OutputT
(some subtype of POutput
).
Common PTransforms include root PTransforms like TextIO.Read
and Create
, processing and conversion operations like
ParDo
, GroupByKey
,
CoGroupByKey
, Combine
, and Count
, and
outputting PTransforms like TextIO.Write
.
New PTransforms can be created by composing existing PTransforms. Most PTransforms in this package are composites, and users can also create composite PTransforms for their own application-specific logic.