public class PCollectionTuple extends java.lang.Object implements PInput, POutput
PCollectionTuple
is an immutable tuple of heterogeneously-typed PCollections
, "keyed" by TupleTags
. A PCollectionTuple
can be used as
the input or output of a PTransform
taking or producing multiple PCollection inputs or
outputs that can be of different types, for instance a ParDo
with multiple outputs.
A PCollectionTuple
can be created and accessed like follows:
PCollection<String> pc1 = ...;
PCollection<Integer> pc2 = ...;
PCollection<Iterable<String>> pc3 = ...;
// Create TupleTags for each of the PCollections to put in the
// PCollectionTuple (the type of the TupleTag enables tracking the
// static type of each of the PCollections in the PCollectionTuple):
TupleTag<String> tag1 = new TupleTag<>();
TupleTag<Integer> tag2 = new TupleTag<>();
TupleTag<Iterable<String>> tag3 = new TupleTag<>();
// Create a PCollectionTuple with three PCollections:
PCollectionTuple pcs =
PCollectionTuple.of(tag1, pc1)
.and(tag2, pc2)
.and(tag3, pc3);
// Create an empty PCollectionTuple:
Pipeline p = ...;
PCollectionTuple pcs2 = PCollectionTuple.empty(p);
// Get PCollections out of a PCollectionTuple, using the same tags
// that were used to put them in:
PCollection<Integer> pcX = pcs.get(tag2);
PCollection<String> pcY = pcs.get(tag1);
PCollection<Iterable<String>> pcZ = pcs.get(tag3);
// Get a map of all PCollections in a PCollectionTuple:
Map<TupleTag<?>, PCollection<?>> allPcs = pcs.getAll();
Modifier and Type | Method and Description |
---|---|
<T> PCollectionTuple |
and(java.lang.String tag,
PCollection<T> pc)
A version of
and(TupleTag, PCollection) that takes in a String instead of a TupleTag. |
<T> PCollectionTuple |
and(TupleTag<T> tag,
PCollection<T> pc)
Returns a new
PCollectionTuple that has each PCollection and TupleTag
of this PCollectionTuple plus the given PCollection associated with the given
TupleTag . |
<OutputT extends POutput> |
apply(PTransform<? super PCollectionTuple,OutputT> t)
Like
apply(String, PTransform) but defaulting to the name of the PTransform . |
<OutputT extends POutput> |
apply(java.lang.String name,
PTransform<? super PCollectionTuple,OutputT> t)
Applies the given
PTransform to this input PCollectionTuple , using name
to identify this specific application of the transform. |
static PCollectionTuple |
empty(Pipeline pipeline)
Returns an empty
PCollectionTuple that is part of the given Pipeline . |
boolean |
equals(@Nullable java.lang.Object other) |
java.util.Map<TupleTag<?>,PValue> |
expand()
|
void |
finishSpecifyingOutput(java.lang.String transformName,
PInput input,
PTransform<?,?> transform)
As part of applying the producing
PTransform , finalizes this output to make it ready
for being used as an input and for running. |
<T> PCollection<T> |
get(java.lang.String tag)
Returns the
PCollection associated with the given tag in this PCollectionTuple . |
<T> PCollection<T> |
get(TupleTag<T> tag)
|
java.util.Map<TupleTag<?>,PCollection<?>> |
getAll()
Returns an immutable Map from
TupleTag to corresponding PCollection , for all
the members of this PCollectionTuple . |
Pipeline |
getPipeline()
|
<T> boolean |
has(java.lang.String tag)
Returns whether this
PCollectionTuple contains a PCollection with the given
tag. |
<T> boolean |
has(TupleTag<T> tag)
Returns whether this
PCollectionTuple contains a PCollection with the given
tag. |
int |
hashCode() |
static <T> PCollectionTuple |
of(java.lang.String tag,
PCollection<T> pc)
A version of
of(TupleTag, PCollection) that takes in a String instead of a TupleTag . |
static <T> PCollectionTuple |
of(java.lang.String tag1,
PCollection<T> pc1,
java.lang.String tag2,
PCollection<T> pc2)
A version of
of(String, PCollection) that takes in two PCollections of the same type. |
static <T> PCollectionTuple |
of(java.lang.String tag1,
PCollection<T> pc1,
java.lang.String tag2,
PCollection<T> pc2,
java.lang.String tag3,
PCollection<T> pc3)
A version of
of(String, PCollection) that takes in three PCollections of the same
type. |
static <T> PCollectionTuple |
of(java.lang.String tag1,
PCollection<T> pc1,
java.lang.String tag2,
PCollection<T> pc2,
java.lang.String tag3,
PCollection<T> pc3,
java.lang.String tag4,
PCollection<T> pc4)
A version of
of(String, PCollection) that takes in four PCollections of the same type. |
static <T> PCollectionTuple |
of(java.lang.String tag1,
PCollection<T> pc1,
java.lang.String tag2,
PCollection<T> pc2,
java.lang.String tag3,
PCollection<T> pc3,
java.lang.String tag4,
PCollection<T> pc4,
java.lang.String tag5,
PCollection<T> pc5)
A version of
of(String, PCollection) that takes in five PCollections of the same type. |
static <T> PCollectionTuple |
of(TupleTag<T> tag,
PCollection<T> pc)
|
static PCollectionTuple |
ofPrimitiveOutputsInternal(Pipeline pipeline,
TupleTagList outputTags,
java.util.Map<TupleTag<?>,Coder<?>> coders,
WindowingStrategy<?,?> windowingStrategy,
PCollection.IsBounded isBounded)
For internal use only; no backwards-compatibility guarantees.
|
public static PCollectionTuple empty(Pipeline pipeline)
PCollectionTuple
that is part of the given Pipeline
.
A PCollectionTuple
containing additional elements can be created by calling and(org.apache.beam.sdk.values.TupleTag<T>, org.apache.beam.sdk.values.PCollection<T>)
on the result.
public static <T> PCollectionTuple of(TupleTag<T> tag, PCollection<T> pc)
PCollectionTuple
containing the given PCollection
keyed by
the given TupleTag
.
A PCollectionTuple
containing additional elements can be created by calling and(org.apache.beam.sdk.values.TupleTag<T>, org.apache.beam.sdk.values.PCollection<T>)
on the result.
public static <T> PCollectionTuple of(java.lang.String tag, PCollection<T> pc)
of(TupleTag, PCollection)
that takes in a String instead of a TupleTag
.
This method is simpler for cases when a typed tuple-tag is not needed to extract a PCollection, for example when using schema transforms.
public static <T> PCollectionTuple of(java.lang.String tag1, PCollection<T> pc1, java.lang.String tag2, PCollection<T> pc2)
of(String, PCollection)
that takes in two PCollections of the same type.public static <T> PCollectionTuple of(java.lang.String tag1, PCollection<T> pc1, java.lang.String tag2, PCollection<T> pc2, java.lang.String tag3, PCollection<T> pc3)
of(String, PCollection)
that takes in three PCollections of the same
type.public static <T> PCollectionTuple of(java.lang.String tag1, PCollection<T> pc1, java.lang.String tag2, PCollection<T> pc2, java.lang.String tag3, PCollection<T> pc3, java.lang.String tag4, PCollection<T> pc4)
of(String, PCollection)
that takes in four PCollections of the same type.public static <T> PCollectionTuple of(java.lang.String tag1, PCollection<T> pc1, java.lang.String tag2, PCollection<T> pc2, java.lang.String tag3, PCollection<T> pc3, java.lang.String tag4, PCollection<T> pc4, java.lang.String tag5, PCollection<T> pc5)
of(String, PCollection)
that takes in five PCollections of the same type.public <T> PCollectionTuple and(TupleTag<T> tag, PCollection<T> pc)
PCollectionTuple
that has each PCollection
and TupleTag
of this PCollectionTuple
plus the given PCollection
associated with the given
TupleTag
.
The given TupleTag
should not already be mapped to a PCollection
in this
PCollectionTuple
.
Each PCollection
in the resulting PCollectionTuple
must be part of the same
Pipeline
.
public <T> PCollectionTuple and(java.lang.String tag, PCollection<T> pc)
and(TupleTag, PCollection)
that takes in a String instead of a TupleTag.
This method is simpler for cases when a typed tuple-tag is not needed to extract a PCollection, for example when using schema transforms.
public <T> boolean has(TupleTag<T> tag)
PCollectionTuple
contains a PCollection
with the given
tag.public <T> boolean has(java.lang.String tag)
PCollectionTuple
contains a PCollection
with the given
tag.public <T> PCollection<T> get(TupleTag<T> tag)
PCollection
associated with the given TupleTag
in this PCollectionTuple
. Throws IllegalArgumentException
if there is no such PCollection
, i.e., !has(tag)
.public <T> PCollection<T> get(java.lang.String tag)
PCollection
associated with the given tag in this PCollectionTuple
.
Throws IllegalArgumentException
if there is no such PCollection
, i.e., !has(tag)
.public java.util.Map<TupleTag<?>,PCollection<?>> getAll()
TupleTag
to corresponding PCollection
, for all
the members of this PCollectionTuple
.public <OutputT extends POutput> OutputT apply(PTransform<? super PCollectionTuple,OutputT> t)
apply(String, PTransform)
but defaulting to the name of the PTransform
.PTransform
public <OutputT extends POutput> OutputT apply(java.lang.String name, PTransform<? super PCollectionTuple,OutputT> t)
PTransform
to this input PCollectionTuple
, using name
to identify this specific application of the transform. This name is used in various places,
including the monitoring UI, logging, and to stably identify this application node in the job
graph.PTransform
@Internal public static PCollectionTuple ofPrimitiveOutputsInternal(Pipeline pipeline, TupleTagList outputTags, java.util.Map<TupleTag<?>,Coder<?>> coders, WindowingStrategy<?,?> windowingStrategy, PCollection.IsBounded isBounded)
Returns a PCollectionTuple
with each of the given tags mapping to a new output
PCollection
.
For use by primitive transformations only.
public Pipeline getPipeline()
PInput
getPipeline
in interface PInput
getPipeline
in interface POutput
public java.util.Map<TupleTag<?>,PValue> expand()
PInput
PInput
into a list of its component output PValues
.
PValue
expands to itself.
PValues
(such as PCollectionTuple
or PCollectionList
) expands to its component PValue PValues
.
Not intended to be invoked directly by user code.
public void finishSpecifyingOutput(java.lang.String transformName, PInput input, PTransform<?,?> transform)
POutput
PTransform
, finalizes this output to make it ready
for being used as an input and for running.
This includes ensuring that all PCollections
have Coders
specified or defaulted.
Automatically invoked whenever this POutput
is output, after POutput.finishSpecifyingOutput(String, PInput, PTransform)
has been called on each component
PValue
returned by POutput.expand()
.
finishSpecifyingOutput
in interface POutput
public boolean equals(@Nullable java.lang.Object other)
equals
in class java.lang.Object
public int hashCode()
hashCode
in class java.lang.Object