@Internal public class DeltaIterationBase<ST,WT> extends DualInputOperator<ST,WT,ST,AbstractRichFunction> implements IterationOperator
BulkIterationBase
,
but maintains state across the individual iteration steps. The state is called the solution set, can be obtained
via getSolutionSet()
, and be accessed by joining (or CoGrouping) with it. The solution
set is updated by producing a delta for it, which is merged into the solution set at the end of each iteration step.
The delta iteration must be closed by setting a delta for the solution set (setSolutionSetDelta(org.apache.flink.api.common.operators.Operator)
)
and the new workset (the data set that will be fed back, setNextWorkset(org.apache.flink.api.common.operators.Operator)
).
The DeltaIteration itself represents the result after the iteration has terminated.
Delta iterations terminate when the feed back data set (the workset) is empty.
In addition, a maximum number of steps is given as a fall back termination guard.
Elements in the solution set are uniquely identified by a key. When merging the solution set delta, contained elements with the same key are replaced.
This class is a subclass of DualInputOperator
. The solution set is considered the first input, the
workset is considered the second input.
Modifier and Type | Class and Description |
---|---|
static class |
DeltaIterationBase.SolutionSetPlaceHolder<ST>
Specialized operator to use as a recognizable place-holder for the solution set input to the
step function.
|
static class |
DeltaIterationBase.WorksetPlaceHolder<WT>
Specialized operator to use as a recognizable place-holder for the working set input to the
step function.
|
input1, input2
broadcastInputs, userFunction
compilerHints, name, operatorInfo, parameters
Constructor and Description |
---|
DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo,
int keyPosition) |
DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo,
int[] keyPositions) |
DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo,
int[] keyPositions,
String name) |
DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo,
int keyPosition,
String name) |
Modifier and Type | Method and Description |
---|---|
protected List<ST> |
executeOnCollections(List<ST> inputData1,
List<WT> inputData2,
RuntimeContext runtimeContext,
ExecutionConfig executionConfig) |
AggregatorRegistry |
getAggregators() |
Map<String,Operator<?>> |
getBroadcastInputs()
DeltaIteration meta operator cannot have broadcast inputs.
|
Operator<ST> |
getInitialSolutionSet()
Returns the initial solution set input, or null, if none is set.
|
Operator<WT> |
getInitialWorkset()
Returns the initial workset input, or null, if none is set.
|
int |
getMaximumNumberOfIterations() |
Operator<WT> |
getNextWorkset()
Gets the contract that has been set as the next workset.
|
Operator<ST> |
getSolutionSet()
Gets the contract that represents the solution set for the step function.
|
Operator<ST> |
getSolutionSetDelta()
Gets the contract that has been set as the solution set delta.
|
int[] |
getSolutionSetKeyFields() |
Operator<WT> |
getWorkset()
Gets the contract that represents the workset for the step function.
|
boolean |
isSolutionSetUnManaged()
gets whether the solution set is in managed or unmanaged memory.
|
void |
setBroadcastVariable(String name,
Operator<?> root)
The DeltaIteration meta operator cannot have broadcast inputs.
|
<X> void |
setBroadcastVariables(Map<String,Operator<X>> inputs)
The DeltaIteration meta operator cannot have broadcast inputs.
|
void |
setInitialSolutionSet(Operator<ST> input)
Sets the given input as the initial solution set.
|
void |
setInitialWorkset(Operator<WT> input)
Sets the given input as the initial workset.
|
void |
setMaximumNumberOfIterations(int maxIterations) |
void |
setNextWorkset(Operator<WT> result)
Sets the contract of the step function that represents the next workset.
|
void |
setSolutionSetDelta(Operator<ST> delta)
Sets the contract of the step function that represents the solution set delta.
|
void |
setSolutionSetUnManaged(boolean solutionSetUnManaged)
Sets whether to keep the solution set in managed memory (safe against heap exhaustion) or unmanaged memory
(objects on heap).
|
accept, addFirstInput, addFirstInputs, addSecondInput, addSecondInputs, clearFirstInput, clearSecondInput, getFirstInput, getKeyColumns, getNumberOfInputs, getOperatorInfo, getSecondInput, getSemanticProperties, setFirstInput, setFirstInput, setFirstInputs, setSecondInput, setSecondInput, setSecondInputs, setSemanticProperties
asArray, emptyClassArray, getUserCodeWrapper
createUnionCascade, createUnionCascade, createUnionCascade, getCompilerHints, getMinResources, getName, getParallelism, getParameters, getPreferredResources, setName, setParallelism, setParameter, setParameter, setParameter, setResources, toString
public DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo, int keyPosition)
public DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo, int[] keyPositions)
public DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo, int keyPosition, String name)
public DeltaIterationBase(BinaryOperatorInformation<ST,WT,ST> operatorInfo, int[] keyPositions, String name)
public int[] getSolutionSetKeyFields()
public void setMaximumNumberOfIterations(int maxIterations)
public int getMaximumNumberOfIterations()
public AggregatorRegistry getAggregators()
getAggregators
in interface IterationOperator
public Operator<ST> getSolutionSet()
public Operator<WT> getWorkset()
public void setNextWorkset(Operator<WT> result)
result
- The contract representing the next workset.public Operator<WT> getNextWorkset()
public void setSolutionSetDelta(Operator<ST> delta)
delta
- The contract representing the solution set delta.public Operator<ST> getSolutionSetDelta()
public Operator<ST> getInitialSolutionSet()
public Operator<WT> getInitialWorkset()
public void setInitialSolutionSet(Operator<ST> input)
input
- The contract to set the initial solution set.public void setInitialWorkset(Operator<WT> input)
input
- The contract to set as the initial workset.public Map<String,Operator<?>> getBroadcastInputs()
getBroadcastInputs
in class AbstractUdfOperator<ST,AbstractRichFunction>
public void setBroadcastVariable(String name, Operator<?> root)
setBroadcastVariable
in class AbstractUdfOperator<ST,AbstractRichFunction>
name
- Ignored.root
- Ignored.public <X> void setBroadcastVariables(Map<String,Operator<X>> inputs)
setBroadcastVariables
in class AbstractUdfOperator<ST,AbstractRichFunction>
inputs
- Ignoredpublic void setSolutionSetUnManaged(boolean solutionSetUnManaged)
solutionSetUnManaged
- True to keep the solution set in unmanaged memory, false to keep it in managed memory.isSolutionSetUnManaged()
public boolean isSolutionSetUnManaged()
setSolutionSetUnManaged(boolean)
protected List<ST> executeOnCollections(List<ST> inputData1, List<WT> inputData2, RuntimeContext runtimeContext, ExecutionConfig executionConfig)
executeOnCollections
in class DualInputOperator<ST,WT,ST,AbstractRichFunction>
Copyright © 2014–2020 The Apache Software Foundation. All rights reserved.