IN
- The type of the elements in the input Transformation
OUT
- The type of the elements that result from this OneInputTransformation
@Internal public class OneInputTransformation<IN,OUT> extends PhysicalTransformation<OUT>
OneInputStreamOperator
to one input Transformation
.bufferTimeout, description, id, name, outputType, typeUsed, UPPER_BOUND_MAX_PARALLELISM
Constructor and Description |
---|
OneInputTransformation(Transformation<IN> input,
String name,
OneInputStreamOperator<IN,OUT> operator,
TypeInformation<OUT> outputType,
int parallelism)
Creates a new
OneInputTransformation from the given input and operator. |
OneInputTransformation(Transformation<IN> input,
String name,
OneInputStreamOperator<IN,OUT> operator,
TypeInformation<OUT> outputType,
int parallelism,
boolean parallelismConfigured) |
OneInputTransformation(Transformation<IN> input,
String name,
StreamOperatorFactory<OUT> operatorFactory,
TypeInformation<OUT> outputType,
int parallelism) |
OneInputTransformation(Transformation<IN> input,
String name,
StreamOperatorFactory<OUT> operatorFactory,
TypeInformation<OUT> outputType,
int parallelism,
boolean parallelismConfigured)
Creates a new
LegacySinkTransformation from the given input Transformation . |
Modifier and Type | Method and Description |
---|---|
List<Transformation<?>> |
getInputs()
Returns the
transformations that are the immediate predecessors of the
current transformation in the transformation graph. |
TypeInformation<IN> |
getInputType()
Returns the
TypeInformation for the elements of the input. |
OneInputStreamOperator<IN,OUT> |
getOperator() |
StreamOperatorFactory<OUT> |
getOperatorFactory()
Returns the
StreamOperatorFactory of this Transformation. |
KeySelector<IN,?> |
getStateKeySelector()
Returns the
KeySelector that must be used for partitioning keyed state in this
Operation. |
TypeInformation<?> |
getStateKeyType() |
List<Transformation<?>> |
getTransitivePredecessors()
Returns all transitive predecessor
Transformation s of this Transformation . |
void |
setChainingStrategy(ChainingStrategy strategy)
Sets the chaining strategy of this
Transformation . |
void |
setStateKeySelector(KeySelector<IN,?> stateKeySelector)
Sets the
KeySelector that must be used for partitioning keyed state of this
operation. |
void |
setStateKeyType(TypeInformation<?> stateKeyType) |
isSupportsConcurrentExecutionAttempts, setSupportsConcurrentExecutionAttempts
declareManagedMemoryUseCaseAtOperatorScope, declareManagedMemoryUseCaseAtSlotScope, equals, getBufferTimeout, getCoLocationGroupKey, getDescription, getId, getManagedMemoryOperatorScopeUseCaseWeights, getManagedMemorySlotScopeUseCases, getMaxParallelism, getMinResources, getName, getNewNodeId, getOutputType, getParallelism, getPreferredResources, getSlotSharingGroup, getUid, getUserProvidedNodeHash, hashCode, isParallelismConfigured, setBufferTimeout, setCoLocationGroupKey, setDescription, setMaxParallelism, setName, setOutputType, setParallelism, setParallelism, setResources, setSlotSharingGroup, setSlotSharingGroup, setUid, setUidHash, toString, updateManagedMemoryStateBackendUseCase
public OneInputTransformation(Transformation<IN> input, String name, OneInputStreamOperator<IN,OUT> operator, TypeInformation<OUT> outputType, int parallelism)
OneInputTransformation
from the given input and operator.input
- The input Transformation
name
- The name of the Transformation
, this will be shown in Visualizations and
the Logoperator
- The TwoInputStreamOperator
outputType
- The type of the elements produced by this OneInputTransformation
parallelism
- The parallelism of this OneInputTransformation
public OneInputTransformation(Transformation<IN> input, String name, OneInputStreamOperator<IN,OUT> operator, TypeInformation<OUT> outputType, int parallelism, boolean parallelismConfigured)
public OneInputTransformation(Transformation<IN> input, String name, StreamOperatorFactory<OUT> operatorFactory, TypeInformation<OUT> outputType, int parallelism)
public OneInputTransformation(Transformation<IN> input, String name, StreamOperatorFactory<OUT> operatorFactory, TypeInformation<OUT> outputType, int parallelism, boolean parallelismConfigured)
LegacySinkTransformation
from the given input Transformation
.input
- The input Transformation
name
- The name of the Transformation
, this will be shown in Visualizations and
the LogoperatorFactory
- The TwoInputStreamOperator
factoryoutputType
- The type of the elements produced by this OneInputTransformation
parallelism
- The parallelism of this OneInputTransformation
parallelismConfigured
- If true, the parallelism of the transformation is explicitly set
and should be respected. Otherwise the parallelism can be changed at runtime.public TypeInformation<IN> getInputType()
TypeInformation
for the elements of the input.@VisibleForTesting public OneInputStreamOperator<IN,OUT> getOperator()
public StreamOperatorFactory<OUT> getOperatorFactory()
StreamOperatorFactory
of this Transformation.public void setStateKeySelector(KeySelector<IN,?> stateKeySelector)
KeySelector
that must be used for partitioning keyed state of this
operation.stateKeySelector
- The KeySelector
to setpublic KeySelector<IN,?> getStateKeySelector()
KeySelector
that must be used for partitioning keyed state in this
Operation.public void setStateKeyType(TypeInformation<?> stateKeyType)
public TypeInformation<?> getStateKeyType()
public List<Transformation<?>> getTransitivePredecessors()
Transformation
Transformation
s of this Transformation
.
This is, for example, used when determining whether a feedback edge of an iteration actually
has the iteration head as a predecessor.getTransitivePredecessors
in class Transformation<OUT>
public List<Transformation<?>> getInputs()
Transformation
transformations
that are the immediate predecessors of the
current transformation in the transformation graph.getInputs
in class Transformation<OUT>
public final void setChainingStrategy(ChainingStrategy strategy)
PhysicalTransformation
Transformation
.setChainingStrategy
in class PhysicalTransformation<OUT>
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.