T
- the type of the elements of the resulting PCollection
public class Create<T>
extends java.lang.Object
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.
Modifier and Type | Class and Description |
---|---|
static class |
Create.OfValueProvider<T>
|
static class |
Create.TimestampedValues<T>
A
PTransform that creates a PCollection whose elements have associated
timestamps. |
static class |
Create.Values<T>
A
PTransform that creates a PCollection from a set of in-memory objects. |
Constructor and Description |
---|
Create() |
Modifier and Type | Method and Description |
---|---|
static <T> Create.Values<T> |
empty(Coder<T> coder)
Returns a new
Create.Values transform that produces an empty PCollection . |
static Create.Values<Row> |
empty(Schema schema)
Returns a new
Create.Values transform that produces an empty PCollection of
rows. |
static <T> Create.Values<T> |
empty(TypeDescriptor<T> type)
Returns a new
Create.Values transform that produces an empty PCollection . |
static <T> Create.Values<T> |
of(java.lang.Iterable<T> elems)
Returns a new
Create.Values transform that produces a PCollection containing
elements of the provided Iterable . |
static <K,V> Create.Values<KV<K,V>> |
of(java.util.Map<K,V> elems)
Returns a new
Create.Values transform that produces a PCollection of KV s corresponding to the keys and values of the specified Map . |
static <T> Create.Values<T> |
of(T elem,
T... elems)
Returns a new
Create.Values transform that produces a PCollection containing
the specified elements. |
static <T> Create.OfValueProvider<T> |
ofProvider(ValueProvider<T> provider,
Coder<T> coder)
Returns an
Create.OfValueProvider transform that produces a PCollection of a single
element provided by the given ValueProvider . |
static <T> Create.TimestampedValues<T> |
timestamped(java.lang.Iterable<T> values,
java.lang.Iterable<java.lang.Long> timestamps)
Returns a new root transform that produces a
PCollection containing the specified
elements with the specified timestamps. |
static <T> Create.TimestampedValues<T> |
timestamped(java.lang.Iterable<TimestampedValue<T>> elems)
Returns a new
Create.TimestampedValues transform that produces a PCollection
containing the elements of the provided Iterable with the specified timestamps. |
static <T> Create.TimestampedValues<T> |
timestamped(TimestampedValue<T> elem,
TimestampedValue<T>... elems)
Returns a new
Create.TimestampedValues transform that produces a PCollection
containing the specified elements with the specified timestamps. |
public static <T> Create.Values<T> of(java.lang.Iterable<T> elems)
Create.Values
transform that produces a PCollection
containing
elements of the provided Iterable
.
The argument should not be modified after this is called.
The elements of the output PCollection
will have a timestamp of negative infinity,
see timestamped(java.lang.Iterable<org.apache.beam.sdk.values.TimestampedValue<T>>)
for a way of creating a PCollection
with timestamped
elements.
By default, Create.Values
can automatically determine the Coder
to use if
all elements have the same non-parameterized run-time class, and a default coder is registered
for that class. See CoderRegistry
for details on how defaults are determined.
Otherwise, use Create.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly.
@SafeVarargs public static <T> Create.Values<T> of(T elem, T... elems)
Create.Values
transform that produces a PCollection
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 a PCollection
with timestamped elements.
The arguments should not be modified after this is called.
By default, Create.Values
can automatically determine the Coder
to use if
all elements have the same non-parameterized run-time class, and a default coder is registered
for that class. See CoderRegistry
for details on how defaults are determined.
Otherwise, use Create.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly.
public static Create.Values<Row> empty(Schema schema)
Create.Values
transform that produces an empty PCollection
of
rows.public static <T> Create.Values<T> empty(Coder<T> coder)
Create.Values
transform that produces an empty PCollection
.
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 a PCollection
with timestamped elements.
Since there are no elements, the Coder
cannot be automatically determined. Instead,
the Coder
is provided via the coder
argument.
public static <T> Create.Values<T> empty(TypeDescriptor<T> type)
Create.Values
transform that produces an empty PCollection
.
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 a PCollection
with timestamped elements.
Since there are no elements, the Coder
cannot be automatically determined. Instead,
the Coder
is determined from given TypeDescriptor<T>
. Note that a default coder
must be registered for the class described in the TypeDescriptor<T>
.
public static <K,V> Create.Values<KV<K,V>> of(java.util.Map<K,V> elems)
Create.Values
transform that produces a PCollection
of KV
s corresponding to the keys and values of the specified Map
.
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 a PCollection
with timestamped elements.
By default, Create.Values
can automatically determine the Coder
to use if
all elements have the same non-parameterized run-time class, and a default coder is registered
for that class. See CoderRegistry
for details on how defaults are determined.
Otherwise, use Create.Values.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder explicitly.
public static <T> Create.OfValueProvider<T> ofProvider(ValueProvider<T> provider, Coder<T> coder)
Create.OfValueProvider
transform that produces a PCollection
of a single
element provided by the given ValueProvider
.public static <T> Create.TimestampedValues<T> timestamped(java.lang.Iterable<TimestampedValue<T>> elems)
Create.TimestampedValues
transform that produces a PCollection
containing the elements of the provided Iterable
with the specified timestamps.
The argument should not be modified after this is called.
By default, Create.TimestampedValues
can automatically determine the Coder
to use if all elements have the same non-parameterized run-time class, and a default coder is
registered for that class. See CoderRegistry
for details on how defaults are
determined. Otherwise, use Create.TimestampedValues.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder
explicitly.
@SafeVarargs public static <T> Create.TimestampedValues<T> timestamped(TimestampedValue<T> elem, TimestampedValue<T>... elems)
Create.TimestampedValues
transform that produces a PCollection
containing the specified elements with the specified timestamps.
The arguments should not be modified after this is called.
public static <T> Create.TimestampedValues<T> timestamped(java.lang.Iterable<T> values, java.lang.Iterable<java.lang.Long> timestamps)
PCollection
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 the Coder
to use if all elements have the same non-parameterized run-time class, and a default coder is
registered for that class. See CoderRegistry
for details on how defaults are
determined. Otherwise, use Create.TimestampedValues.withCoder(org.apache.beam.sdk.coders.Coder<T>)
to set the coder
explicitly.
java.lang.IllegalArgumentException
- if there are a different number of values and timestamps