Class Create<T>
- Type Parameters:
T
- the type of the elements of the resultingPCollection
Create<T>
takes a collection of elements of type T
known when the pipeline is
constructed and returns a PCollection<T>
containing the elements.
Example of use:
Pipeline p = ...;
PCollection<Integer> pc = p.apply(Create.of(3, 4, 5).withCoder(BigEndianIntegerCoder.of()));
Map<String, Integer> map = ...;
PCollection<KV<String, Integer>> pt =
p.apply(Create.of(map)
.withCoder(KvCoder.of(StringUtf8Coder.of(),
BigEndianIntegerCoder.of())));
Create
can automatically determine the Coder
to use if all elements have the
same run-time class, and a default coder is registered for that class. See CoderRegistry
for details on how defaults are determined.
If a coder can not be inferred, Create.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
must be called explicitly to
set the encoding of the resulting PCollection
.
A good use for Create
is when a PCollection
needs to be created without
dependencies on files or other external entities. This is especially useful during testing.
Caveat: Create
only supports small in-memory datasets.
-
Nested Class Summary
Nested ClassesModifier and TypeClassDescriptionstatic class
static class
APTransform
that creates aPCollection
whose elements have associated timestamps.static class
APTransform
that creates aPCollection
from a set of in-memory objects.static class
APTransform
that creates aPCollection
whose elements have associated windowing metadata. -
Constructor Summary
Constructors -
Method Summary
Modifier and TypeMethodDescriptionstatic <T> Create.Values
<T> Returns a newCreate.Values
transform that produces an emptyPCollection
.static Create.Values
<Row> Returns a newCreate.Values
transform that produces an emptyPCollection
of rows.static <T> Create.Values
<T> empty
(TypeDescriptor<T> type) Returns a newCreate.Values
transform that produces an emptyPCollection
.static <T> Create.Values
<T> Returns a newCreate.Values
transform that produces aPCollection
containing the specified elements.static <T> Create.Values
<T> Returns a newCreate.Values
transform that produces aPCollection
containing elements of the providedIterable
.static <K,
V> Create.Values <KV<K, V>> Returns a newCreate.Values
transform that produces aPCollection
ofKV
s corresponding to the keys and values of the specifiedMap
.static <T> Create.OfValueProvider
<T> ofProvider
(ValueProvider<T> provider, Coder<T> coder) Returns anCreate.OfValueProvider
transform that produces aPCollection
of a single element provided by the givenValueProvider
.static <T> Create.TimestampedValues
<T> timestamped
(Iterable<TimestampedValue<T>> elems) Returns a newCreate.TimestampedValues
transform that produces aPCollection
containing the elements of the providedIterable
with the specified timestamps.static <T> Create.TimestampedValues
<T> timestamped
(Iterable<T> values, Iterable<Long> timestamps) Returns a new root transform that produces aPCollection
containing the specified elements with the specified timestamps.static <T> Create.TimestampedValues
<T> timestamped
(TimestampedValue<T> elem, TimestampedValue<T>... elems) Returns a newCreate.TimestampedValues
transform that produces aPCollection
containing the specified elements with the specified timestamps.static <T> Create.WindowedValues
<T> windowedValues
(Iterable<WindowedValue<T>> elems) Returns a newCreate.WindowedValues
transform that produces aPCollection
containing the elements of the providedIterable
with the specified windowing metadata.static <T> Create.WindowedValues
<T> windowedValues
(WindowedValue<T> elem, WindowedValue<T>... elems) Returns a newCreate.WindowedValues
transform that produces aPCollection
containing the specified elements with the specified windowing metadata.
-
Constructor Details
-
Create
public Create()
-
-
Method Details
-
of
Returns a newCreate.Values
transform that produces aPCollection
containing elements of the providedIterable
.The argument should not be modified after this is called.
The elements of the output
PCollection
will have a timestamp of negative infinity, seetimestamped(java.lang.Iterable<org.apache.beam.sdk.values.TimestampedValue<T>>)
for a way of creating aPCollection
with timestamped elements.By default,
Create.Values
can automatically determine theCoder
to use if all elements have the same non-parameterized run-time class, and a default coder is registered for that class. SeeCoderRegistry
for details on how defaults are determined. Otherwise, useCreate.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly. -
of
Returns a newCreate.Values
transform that produces aPCollection
containing the specified elements.The elements will have a timestamp of negative infinity, see
timestamped(java.lang.Iterable<org.apache.beam.sdk.values.TimestampedValue<T>>)
for a way of creating aPCollection
with timestamped elements.The arguments should not be modified after this is called.
By default,
Create.Values
can automatically determine theCoder
to use if all elements have the same non-parameterized run-time class, and a default coder is registered for that class. SeeCoderRegistry
for details on how defaults are determined. Otherwise, useCreate.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly. -
empty
Returns a newCreate.Values
transform that produces an emptyPCollection
of rows. -
empty
Returns a newCreate.Values
transform that produces an emptyPCollection
.The elements will have a timestamp of negative infinity, see
timestamped(java.lang.Iterable<org.apache.beam.sdk.values.TimestampedValue<T>>)
for a way of creating aPCollection
with timestamped elements.Since there are no elements, the
Coder
cannot be automatically determined. Instead, theCoder
is provided via thecoder
argument. -
empty
Returns a newCreate.Values
transform that produces an emptyPCollection
.The elements will have a timestamp of negative infinity, see
timestamped(java.lang.Iterable<org.apache.beam.sdk.values.TimestampedValue<T>>)
for a way of creating aPCollection
with timestamped elements.Since there are no elements, the
Coder
cannot be automatically determined. Instead, theCoder
is determined from givenTypeDescriptor<T>
. Note that a default coder must be registered for the class described in theTypeDescriptor<T>
. -
of
Returns a newCreate.Values
transform that produces aPCollection
ofKV
s corresponding to the keys and values of the specifiedMap
.The elements will have a timestamp of negative infinity, see
timestamped(java.lang.Iterable<org.apache.beam.sdk.values.TimestampedValue<T>>)
for a way of creating aPCollection
with timestamped elements.By default,
Create.Values
can automatically determine theCoder
to use if all elements have the same non-parameterized run-time class, and a default coder is registered for that class. SeeCoderRegistry
for details on how defaults are determined. Otherwise, useCreate.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly. -
ofProvider
Returns anCreate.OfValueProvider
transform that produces aPCollection
of a single element provided by the givenValueProvider
. -
timestamped
Returns a newCreate.TimestampedValues
transform that produces aPCollection
containing the elements of the providedIterable
with the specified timestamps.The argument should not be modified after this is called.
By default,
Create.TimestampedValues
can automatically determine theCoder
to use if all elements have the same non-parameterized run-time class, and a default coder is registered for that class. SeeCoderRegistry
for details on how defaults are determined. Otherwise, useCreate.TimestampedValues.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly. -
timestamped
@SafeVarargs public static <T> Create.TimestampedValues<T> timestamped(TimestampedValue<T> elem, TimestampedValue<T>... elems) Returns a newCreate.TimestampedValues
transform that produces aPCollection
containing the specified elements with the specified timestamps.The arguments should not be modified after this is called.
-
windowedValues
Returns a newCreate.WindowedValues
transform that produces aPCollection
containing the elements of the providedIterable
with the specified windowing metadata.The argument should not be modified after this is called.
By default,
Create.WindowedValues
can automatically determine theCoder
to use if all elements have the same non-parameterized run-time class, and a default coder is registered for that class. SeeCoderRegistry
for details on how defaults are determined. Otherwise, useCreate.WindowedValues.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly.Likewise, the window coder can be inferred if the window type is registered with the
CoderRegistry
. Otherwise, useCreate.WindowedValues.withWindowCoder(Coder)
to set the window coder explicitly. -
windowedValues
@SafeVarargs public static <T> Create.WindowedValues<T> windowedValues(WindowedValue<T> elem, WindowedValue<T>... elems) Returns a newCreate.WindowedValues
transform that produces aPCollection
containing the specified elements with the specified windowing metadata.The arguments should not be modified after this is called.
-
timestamped
public static <T> Create.TimestampedValues<T> timestamped(Iterable<T> values, Iterable<Long> timestamps) Returns a new root transform that produces aPCollection
containing the specified elements with the specified timestamps.The arguments should not be modified after this is called.
By default,
Create.TimestampedValues
can automatically determine theCoder
to use if all elements have the same non-parameterized run-time class, and a default coder is registered for that class. SeeCoderRegistry
for details on how defaults are determined. Otherwise, useCreate.TimestampedValues.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly.- Throws:
IllegalArgumentException
- if there are a different number of values and timestamps
-