提交 de8e066c 编写于 作者: F Fabian Hueske

[FLINK-1328] [api-breaking][java-api][scala-api][optimizer] Reworked semantic...

[FLINK-1328] [api-breaking][java-api][scala-api][optimizer] Reworked semantic annotations for functions.
- Renamed constantField annotations to forwardedFields annotation
- Forwarded fields can be defined for (nested) tuples, Pojos, case classes
- Added semantic function information to example programs

This closes #311
上级 78f41e9c
......@@ -350,7 +350,7 @@ public class VertexCentricIteration<VertexKey extends Comparable<VertexKey>, Ver
}
// let the operator know that we preserve the key field
updates.withConstantSetFirst("0").withConstantSetSecond("0");
updates.withForwardedFieldsFirst("0").withForwardedFieldsSecond("0");
return iteration.closeWith(updates, updates);
......
......@@ -100,8 +100,8 @@ public class SpargelTranslationTest {
// validate that the semantic properties are set as they should
TwoInputUdfOperator<?, ?, ?, ?> solutionSetJoin = (TwoInputUdfOperator<?, ?, ?, ?>) resultSet.getNextWorkset();
assertTrue(solutionSetJoin.getSemanticProperties().getForwardedField1(0).contains(0));
assertTrue(solutionSetJoin.getSemanticProperties().getForwardedField2(0).contains(0));
assertTrue(solutionSetJoin.getSemanticProperties().getForwardingTargetFields(0, 0).contains(0));
assertTrue(solutionSetJoin.getSemanticProperties().getForwardingTargetFields(1, 0).contains(0));
TwoInputUdfOperator<?, ?, ?, ?> edgesJoin = (TwoInputUdfOperator<?, ?, ?, ?>) solutionSetJoin.getInput1();
......@@ -179,8 +179,8 @@ public class SpargelTranslationTest {
// validate that the semantic properties are set as they should
TwoInputUdfOperator<?, ?, ?, ?> solutionSetJoin = (TwoInputUdfOperator<?, ?, ?, ?>) resultSet.getNextWorkset();
assertTrue(solutionSetJoin.getSemanticProperties().getForwardedField1(0).contains(0));
assertTrue(solutionSetJoin.getSemanticProperties().getForwardedField2(0).contains(0));
assertTrue(solutionSetJoin.getSemanticProperties().getForwardingTargetFields(0, 0).contains(0));
assertTrue(solutionSetJoin.getSemanticProperties().getForwardingTargetFields(1, 0).contains(0));
TwoInputUdfOperator<?, ?, ?, ?> edgesJoin = (TwoInputUdfOperator<?, ?, ?, ?>) solutionSetJoin.getInput1();
......
......@@ -19,7 +19,6 @@ package org.apache.flink.streaming.api.function.aggregation;
import java.lang.reflect.Array;
import java.lang.reflect.Field;
import java.util.ArrayList;
import java.util.List;
import org.apache.flink.api.common.typeinfo.BasicArrayTypeInfo;
......@@ -189,9 +188,7 @@ public abstract class ComparableAggregator<T> extends AggregationFunction<T> {
@SuppressWarnings("unchecked")
CompositeType<T> cType = (CompositeType<T>) typeInfo;
List<FlatFieldDescriptor> fieldDescriptors = new ArrayList<FlatFieldDescriptor>();
cType.getKey(field, 0, fieldDescriptors);
List<FlatFieldDescriptor> fieldDescriptors = cType.getFlatFields(field);
int logicalKeyPosition = fieldDescriptors.get(0).getPosition();
if (cType instanceof PojoTypeInfo) {
......
......@@ -19,7 +19,6 @@ package org.apache.flink.streaming.api.function.aggregation;
import java.lang.reflect.Array;
import java.lang.reflect.Field;
import java.util.ArrayList;
import java.util.List;
import org.apache.flink.api.common.functions.ReduceFunction;
......@@ -138,8 +137,7 @@ public abstract class SumAggregator {
@SuppressWarnings("unchecked")
CompositeType<T> cType = (CompositeType<T>) type;
List<FlatFieldDescriptor> fieldDescriptors = new ArrayList<FlatFieldDescriptor>();
cType.getKey(field, 0, fieldDescriptors);
List<FlatFieldDescriptor> fieldDescriptors = cType.getFlatFields(field);
int logicalKeyPosition = fieldDescriptors.get(0).getPosition();
Class<?> keyClass = fieldDescriptors.get(0).getType().getTypeClass();
......
......@@ -25,6 +25,7 @@ import java.util.Map;
import org.apache.flink.api.common.operators.Operator;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SemanticProperties.EmptySemanticProperties;
import org.apache.flink.compiler.DataStatistics;
import org.apache.flink.compiler.costs.CostEstimator;
import org.apache.flink.compiler.plan.PlanNode;
......@@ -88,7 +89,7 @@ public abstract class AbstractPartialSolutionNode extends OptimizerNode {
@Override
public SemanticProperties getSemanticProperties() {
return null;
return new EmptySemanticProperties();
}
@Override
......
......@@ -24,9 +24,9 @@ import java.util.HashSet;
import java.util.List;
import java.util.Set;
import org.apache.flink.api.common.operators.DualInputSemanticProperties;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.Union;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.compiler.CompilerException;
import org.apache.flink.compiler.DataStatistics;
import org.apache.flink.compiler.costs.CostEstimator;
......@@ -255,9 +255,7 @@ public class BinaryUnionNode extends TwoInputNode {
@Override
public SemanticProperties getSemanticProperties() {
DualInputSemanticProperties sprops = new DualInputSemanticProperties();
sprops.setAllFieldsConstant(true);
return sprops;
return new UnionSemanticProperties();
}
@Override
......@@ -270,4 +268,35 @@ public class BinaryUnionNode extends TwoInputNode {
this.estimatedOutputSize = in1.estimatedOutputSize > 0 && in2.estimatedOutputSize > 0 ?
in1.estimatedOutputSize + in2.estimatedOutputSize : -1;
}
public static class UnionSemanticProperties implements SemanticProperties {
@Override
public FieldSet getForwardingTargetFields(int input, int sourceField) {
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException("Invalid input index for binary union node.");
}
return new FieldSet(sourceField);
}
@Override
public int getForwardingSourceField(int input, int targetField) {
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
}
return targetField;
}
@Override
public FieldSet getReadFields(int input) {
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
}
return FieldSet.EMPTY_SET;
}
}
}
......@@ -25,6 +25,7 @@ import java.util.List;
import java.util.Set;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SemanticProperties.EmptySemanticProperties;
import org.apache.flink.api.common.operators.base.BulkIterationBase;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.compiler.CompilerException;
......@@ -186,7 +187,7 @@ public class BulkIterationNode extends SingleInputNode implements IterationNode
@Override
public SemanticProperties getSemanticProperties() {
return null;
return new EmptySemanticProperties();
}
protected void readStubAnnotations() {}
......
......@@ -29,6 +29,7 @@ import org.apache.flink.api.common.operators.GenericDataSinkBase;
import org.apache.flink.api.common.operators.Operator;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SemanticProperties.EmptySemanticProperties;
import org.apache.flink.compiler.CompilerException;
import org.apache.flink.compiler.DataStatistics;
import org.apache.flink.compiler.costs.CostEstimator;
......@@ -234,7 +235,7 @@ public class DataSinkNode extends OptimizerNode {
@Override
public SemanticProperties getSemanticProperties() {
return null;
return new EmptySemanticProperties();
}
// --------------------------------------------------------------------------------------------
......
......@@ -31,6 +31,7 @@ import org.apache.flink.api.common.io.statistics.BaseStatistics;
import org.apache.flink.api.common.operators.GenericDataSourceBase;
import org.apache.flink.api.common.operators.Operator;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SemanticProperties.EmptySemanticProperties;
import org.apache.flink.compiler.DataStatistics;
import org.apache.flink.compiler.PactCompiler;
import org.apache.flink.compiler.costs.CostEstimator;
......@@ -195,7 +196,7 @@ public class DataSourceNode extends OptimizerNode {
@Override
public SemanticProperties getSemanticProperties() {
return null;
return new EmptySemanticProperties();
}
@Override
......
......@@ -52,11 +52,7 @@ public class FilterNode extends SingleInputNode {
@Override
public SemanticProperties getSemanticProperties() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
sprops.setAllFieldsConstant(true);
return sprops;
return new SingleInputSemanticProperties.AllFieldsForwardedProperties();
}
@Override
......
......@@ -666,35 +666,7 @@ public abstract class OptimizerNode implements Visitable<OptimizerNode>, Estimat
// ------------------------------------------------------------------------
// Access of stub annotations
// ------------------------------------------------------------------------
/**
* Returns the key columns for the specific input, if all keys are preserved
* by this node. Null, otherwise.
*/
protected int[] getConstantKeySet(int input) {
Operator<?> contract = getPactContract();
if (contract instanceof AbstractUdfOperator<?, ?>) {
AbstractUdfOperator<?, ?> abstractPact = (AbstractUdfOperator<?, ?>) contract;
int[] keyColumns = abstractPact.getKeyColumns(input);
if (keyColumns != null) {
if (keyColumns.length == 0) {
return null;
}
for (int keyColumn : keyColumns) {
FieldSet fs = getSemanticProperties() == null ? null : getSemanticProperties().getForwardFields(input, keyColumn);
if (fs == null) {
return null;
} else if (!fs.contains(keyColumn)) {
return null;
}
}
return keyColumns;
}
}
return null;
}
/**
* An optional method where nodes can describe which fields will be unique in their output.
*/
......
......@@ -23,6 +23,8 @@ import java.util.Collections;
import java.util.List;
import org.apache.flink.api.common.functions.Partitioner;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SingleInputSemanticProperties;
import org.apache.flink.api.common.operators.base.PartitionOperatorBase;
import org.apache.flink.api.common.operators.base.PartitionOperatorBase.PartitionMethod;
import org.apache.flink.api.common.operators.util.FieldSet;
......@@ -74,9 +76,8 @@ public class PartitionNode extends SingleInputNode {
}
@Override
public boolean isFieldConstant(int input, int fieldNumber) {
// Partition does not change any data
return true;
public SemanticProperties getSemanticProperties() {
return new SingleInputSemanticProperties.AllFieldsForwardedProperties();
}
// --------------------------------------------------------------------------------------------
......
......@@ -59,9 +59,7 @@ public class UnaryOperatorNode extends SingleInputNode {
@Override
public SemanticProperties getSemanticProperties() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
sprops.setAllFieldsConstant(true);
return sprops;
return new SingleInputSemanticProperties.AllFieldsForwardedProperties();
}
@Override
......
......@@ -25,6 +25,7 @@ import java.util.List;
import java.util.Set;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SemanticProperties.EmptySemanticProperties;
import org.apache.flink.api.common.operators.base.DeltaIterationBase;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.typeinfo.NothingTypeInfo;
......@@ -222,7 +223,7 @@ public class WorksetIterationNode extends TwoInputNode implements IterationNode
@Override
public SemanticProperties getSemanticProperties() {
return null;
return new EmptySemanticProperties();
}
protected void readStubAnnotations() {}
......
......@@ -19,7 +19,6 @@
package org.apache.flink.compiler.dataproperties;
import java.util.HashSet;
import java.util.Iterator;
import java.util.Set;
import org.apache.flink.api.common.functions.Partitioner;
......@@ -32,6 +31,8 @@ import org.apache.flink.compiler.CompilerException;
import org.apache.flink.compiler.plan.Channel;
import org.apache.flink.compiler.util.Utils;
import org.apache.flink.runtime.operators.shipping.ShipStrategyType;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* This class represents global properties of the data at a certain point in the plan.
......@@ -41,6 +42,8 @@ import org.apache.flink.runtime.operators.shipping.ShipStrategyType;
* or an FieldSet with the hash partitioning columns.
*/
public class GlobalProperties implements Cloneable {
public static final Logger LOG = LoggerFactory.getLogger(GlobalProperties.class);
private PartitioningProperty partitioning; // the type partitioning
......@@ -213,18 +216,6 @@ public class GlobalProperties implements Cloneable {
}
}
public Ordering getOrdering() {
return this.ordering;
}
public void setOrdering(Ordering ordering) {
this.ordering = ordering;
}
public void setPartitioningFields(FieldList partitioningFields) {
this.partitioningFields = partitioningFields;
}
public boolean isFullyReplicated() {
return this.partitioning == PartitioningProperty.FULL_REPLICATION;
}
......@@ -246,83 +237,114 @@ public class GlobalProperties implements Cloneable {
}
/**
* Filters these GlobalProperties by the fields that are constant or forwarded to another output field.
* Filters these GlobalProperties by the fields that are forwarded to the output
* as described by the SemanticProperties.
*
* @param props The node representing the contract.
* @param props The semantic properties holding information about forwarded fields.
* @param input The index of the input.
* @return The filtered GlobalProperties
*/
public GlobalProperties filterBySemanticProperties(SemanticProperties props, int input) {
// check if partitioning survives
FieldList forwardFields = null;
GlobalProperties returnProps = this;
if (props == null) {
return new GlobalProperties();
throw new NullPointerException("SemanticProperties may not be null.");
}
if (this.ordering != null) {
Ordering no = new Ordering();
for (int index : this.ordering.getInvolvedIndexes()) {
forwardFields = props.getForwardFields(input, index) == null ? null: props.getForwardFields(input, index).toFieldList();
if (forwardFields == null) {
returnProps = new GlobalProperties();
no = null;
break;
} else {
returnProps = returnProps == this ? this.clone() : returnProps;
for (int i = 0; i < forwardFields.size(); i++) {
no.appendOrdering(forwardFields.get(i), this.ordering.getType(index), this.ordering.getOrder(index));
GlobalProperties gp = new GlobalProperties();
// filter partitioning
switch(this.partitioning) {
case FULL_REPLICATION:
return gp;
case RANGE_PARTITIONED:
// check if ordering is preserved
Ordering newOrdering = new Ordering();
for (int i = 0; i < this.ordering.getInvolvedIndexes().size(); i++) {
int sourceField = this.ordering.getInvolvedIndexes().get(i);
FieldSet targetField = props.getForwardingTargetFields(input, sourceField);
if (targetField == null || targetField.size() == 0) {
// partitioning is destroyed
newOrdering = null;
break;
} else {
// use any field of target fields for now. We should use something like field equivalence sets in the future.
if(targetField.size() > 1) {
LOG.warn("Found that a field is forwarded to more than one target field in " +
"semantic forwarded field information. Will only use the field with the lowest index.");
}
newOrdering.appendOrdering(targetField.toArray()[0], this.ordering.getType(i), this.ordering.getOrder(i));
}
}
returnProps.setOrdering(no);
}
}
if (this.partitioningFields != null) {
returnProps = returnProps == this ? this.clone() : returnProps;
returnProps.setPartitioningFields(new FieldList());
if(newOrdering != null) {
gp.partitioning = PartitioningProperty.RANGE_PARTITIONED;
gp.ordering = newOrdering;
gp.partitioningFields = newOrdering.getInvolvedIndexes();
}
break;
case HASH_PARTITIONED:
case ANY_PARTITIONING:
case CUSTOM_PARTITIONING:
FieldList newPartitioningFields = new FieldList();
for (int sourceField : this.partitioningFields) {
FieldSet targetField = props.getForwardingTargetFields(input, sourceField);
for (int index : this.partitioningFields) {
forwardFields = props.getForwardFields(input, index) == null ? null: props.getForwardFields(input, index).toFieldList();
if (forwardFields == null) {
returnProps = new GlobalProperties();
break;
} else {
returnProps.setPartitioningFields(returnProps.getPartitioningFields().addFields(forwardFields));
if (targetField == null || targetField.size() == 0) {
newPartitioningFields = null;
break;
} else {
// use any field of target fields for now. We should use something like field equivalence sets in the future.
if(targetField.size() > 1) {
LOG.warn("Found that a field is forwarded to more than one target field in " +
"semantic forwarded field information. Will only use the field with the lowest index.");
}
newPartitioningFields = newPartitioningFields.addField(targetField.toArray()[0]);
}
}
}
if(newPartitioningFields != null) {
gp.partitioning = this.partitioning;
gp.partitioningFields = newPartitioningFields;
gp.customPartitioner = this.customPartitioner;
}
break;
case FORCED_REBALANCED:
case RANDOM:
gp.partitioning = this.partitioning;
break;
default:
throw new RuntimeException("Unknown partitioning type.");
}
// filter unique field combinations
if (this.uniqueFieldCombinations != null) {
HashSet<FieldSet> newSet = new HashSet<FieldSet>();
newSet.addAll(this.uniqueFieldCombinations);
for (Iterator<FieldSet> combos = this.uniqueFieldCombinations.iterator(); combos.hasNext(); ){
FieldSet current = combos.next();
FieldSet nfs = new FieldSet();
for (Integer field : current) {
if (props.getForwardFields(input, field) == null) {
newSet.remove(current);
nfs = null;
Set<FieldSet> newUniqueFieldCombinations = new HashSet<FieldSet>();
for (FieldSet fieldCombo : this.uniqueFieldCombinations) {
FieldSet newFieldCombo = new FieldSet();
for (Integer sourceField : fieldCombo) {
FieldSet targetField = props.getForwardingTargetFields(input, sourceField);
if (targetField == null || targetField.size() == 0) {
newFieldCombo = null;
break;
} else {
nfs = nfs.addFields(props.getForwardFields(input, field));
// use any field of target fields for now. We should use something like field equivalence sets in the future.
if(targetField.size() > 1) {
LOG.warn("Found that a field is forwarded to more than one target field in " +
"semantic forwarded field information. Will only use the field with the lowest index.");
}
newFieldCombo = newFieldCombo.addField(targetField.toArray()[0]);
}
}
if (nfs != null) {
newSet.remove(current);
newSet.add(nfs);
if (newFieldCombo != null) {
newUniqueFieldCombinations.add(newFieldCombo);
}
}
GlobalProperties gp = returnProps.clone();
gp.uniqueFieldCombinations = newSet.isEmpty() ? null : newSet;
return gp;
}
if (this.partitioning == PartitioningProperty.FULL_REPLICATION) {
return new GlobalProperties();
if(!newUniqueFieldCombinations.isEmpty()) {
gp.uniqueFieldCombinations = newUniqueFieldCombinations;
}
}
return returnProps;
return gp;
}
......
......@@ -94,11 +94,11 @@ public class InterestingProperties implements Cloneable
public InterestingProperties filterByCodeAnnotations(OptimizerNode node, int input)
{
InterestingProperties iProps = new InterestingProperties();
SemanticProperties props = null;
if (node instanceof SingleInputNode) {
props = node.getSemanticProperties();
} else if (node instanceof TwoInputNode) {
SemanticProperties props;
if (node instanceof SingleInputNode || node instanceof TwoInputNode) {
props = node.getSemanticProperties();
} else {
props = new SemanticProperties.EmptySemanticProperties();
}
for (RequestedGlobalProperties rgp : this.globalProps) {
......
......@@ -20,20 +20,23 @@
package org.apache.flink.compiler.dataproperties;
import java.util.HashSet;
import java.util.Iterator;
import java.util.Set;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* This class represents local properties of the data. A local property is a property that exists
* within the data of a single partition.
*/
public class LocalProperties implements Cloneable {
public static final Logger LOG = LoggerFactory.getLogger(GlobalProperties.class);
public static final LocalProperties EMPTY = new LocalProperties();
// --------------------------------------------------------------------------------------------
......@@ -126,101 +129,107 @@ public class LocalProperties implements Cloneable {
// --------------------------------------------------------------------------------------------
/**
* Filters these properties by what can be preserved through a user function's constant fields set.
*
* @param props The optimizer node that potentially modifies the properties.
* @param input The input of the node which is relevant.
* Filters these LocalProperties by the fields that are forwarded to the output
* as described by the SemanticProperties.
*
* @param props The semantic properties holding information about forwarded fields.
* @param input The index of the input.
* @return The filtered LocalProperties
*/
public LocalProperties filterBySemanticProperties(SemanticProperties props, int input) {
// check, whether the local order is preserved
Ordering no = this.ordering;
FieldList ngf = this.groupedFields;
Set<FieldSet> nuf = this.uniqueFields;
FieldList forwardList = null;
if (props == null) {
return new LocalProperties();
throw new NullPointerException("SemanticProperties may not be null.");
}
LocalProperties returnProps = new LocalProperties();
// check if sorting is preserved
if (this.ordering != null) {
no = new Ordering();
final FieldList involvedIndexes = this.ordering.getInvolvedIndexes();
for (int i = 0; i < involvedIndexes.size(); i++) {
forwardList = props.getForwardFields(input, involvedIndexes.get(i)) == null ? null : props.getForwardFields(input, involvedIndexes.get(i)).toFieldList();
Ordering newOrdering = new Ordering();
if (forwardList == null) {
no = null;
ngf = null;
/*if (i == 0) {
no = null;
ngf = null;
for (int i = 0; i < this.ordering.getInvolvedIndexes().size(); i++) {
int sourceField = this.ordering.getInvolvedIndexes().get(i);
FieldSet targetField = props.getForwardingTargetFields(input, sourceField);
if (targetField == null || targetField.size() == 0) {
if (i == 0) {
// order fully destroyed
newOrdering = null;
break;
} else {
no = this.ordering.createNewOrderingUpToIndex(i);
ngf = no.getInvolvedIndexes();
}*/
break;
// order partially preserved
break;
}
} else {
no.appendOrdering(forwardList.get(0), this.ordering.getType(i), this.ordering.getOrder(i));
ngf = no.getInvolvedIndexes();
// use any field of target fields for now. We should use something like field equivalence sets in the future.
if(targetField.size() > 1) {
LOG.warn("Found that a field is forwarded to more than one target field in " +
"semantic forwarded field information. Will only use the field with the lowest index.");
}
newOrdering.appendOrdering(targetField.toArray()[0], this.ordering.getType(i), this.ordering.getOrder(i));
}
}
returnProps.ordering = newOrdering;
if (newOrdering != null) {
returnProps.groupedFields = newOrdering.getInvolvedIndexes();
} else {
returnProps.groupedFields = null;
}
}
// check if grouping is preserved
else if (this.groupedFields != null) {
// check, whether the local key grouping is preserved
for (Integer index : this.groupedFields) {
forwardList = props.getForwardFields(input, index) == null ? null : props.getForwardFields(input, index).toFieldList();
if (forwardList == null) {
ngf = null;
FieldList newGroupedFields = new FieldList();
for (Integer sourceField : this.groupedFields) {
FieldSet targetField = props.getForwardingTargetFields(input, sourceField);
if (targetField == null || targetField.size() == 0) {
newGroupedFields = null;
break;
} else if (!forwardList.contains(index)) {
FieldList grouped = new FieldList();
for (Integer value : ngf.toFieldList()) {
if (value.intValue() == index) {
grouped = grouped.addFields(forwardList);
} else {
grouped = grouped.addField(value);
}
} else {
// use any field of target fields for now. We should use something like field equivalence sets in the future.
if(targetField.size() > 1) {
LOG.warn("Found that a field is forwarded to more than one target field in " +
"semantic forwarded field information. Will only use the field with the lowest index.");
}
ngf = grouped;
newGroupedFields = newGroupedFields.addField(targetField.toArray()[0]);
}
}
returnProps.groupedFields = newGroupedFields;
}
if (this.uniqueFields != null) {
HashSet<FieldSet> newSet = new HashSet<FieldSet>();
newSet.addAll(this.uniqueFields);
for (Iterator<FieldSet> combos = this.uniqueFields.iterator(); combos.hasNext(); ){
FieldSet current = combos.next();
FieldSet nfs = new FieldSet();
for (Integer field : current) {
if (props.getForwardFields(input, field) == null) {
newSet.remove(current);
nfs = null;
Set<FieldSet> newUniqueFields = new HashSet<FieldSet>();
for (FieldSet fields : this.uniqueFields) {
FieldSet newFields = new FieldSet();
for (Integer sourceField : fields) {
FieldSet targetField = props.getForwardingTargetFields(input, sourceField);
if (targetField == null || targetField.size() == 0) {
newFields = null;
break;
} else {
nfs = nfs.addFields(props.getForwardFields(input, field));
// use any field of target fields for now. We should use something like field equivalence sets in the future.
if(targetField.size() > 1) {
LOG.warn("Found that a field is forwarded to more than one target field in " +
"semantic forwarded field information. Will only use the field with the lowest index.");
}
newFields = newFields.addField(targetField.toArray()[0]);
}
}
if (nfs != null) {
newSet.remove(current);
newSet.add(nfs);
if (newFields != null) {
newUniqueFields.add(newFields);
}
}
nuf = newSet.isEmpty() ? null : newSet;
if (!newUniqueFields.isEmpty()) {
returnProps.uniqueFields = newUniqueFields;
} else {
returnProps.uniqueFields = null;
}
}
if (no == this.ordering && ngf == this.groupedFields && nuf == this.uniqueFields) {
return this;
} else {
LocalProperties lp = new LocalProperties();
lp.ordering = no;
lp.groupedFields = ngf;
lp.uniqueFields = nuf;
return lp;
}
return returnProps;
}
// --------------------------------------------------------------------------------------------
......
......@@ -22,7 +22,6 @@ import org.apache.flink.api.common.distributions.DataDistribution;
import org.apache.flink.api.common.functions.Partitioner;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.compiler.CompilerException;
import org.apache.flink.compiler.plan.Channel;
......@@ -189,73 +188,65 @@ public final class RequestedGlobalProperties implements Cloneable {
this.customPartitioner = null;
}
public void setPartitioningFields(FieldSet partitioned) {
this.partitioningFields = partitioned;
}
public void setPartitioningFields(FieldSet fields, PartitioningProperty partitioning) {
this.partitioningFields = fields;
this.partitioning = partitioning;
}
public void setOrdering(Ordering newOrdering) {
this.ordering = newOrdering;
}
/**
* Filters these properties by what can be preserved by the given node when propagated down
* Filters these properties by what can be preserved by the given SemanticProperties when propagated down
* to the given input.
*
* @param props The node representing the contract.
* @param input The index of the input.
* @param props The SemanticProperties which define which fields are preserved.
* @param input The index of the operator's input.
* @return The filtered RequestedGlobalProperties
*/
public RequestedGlobalProperties filterBySemanticProperties(SemanticProperties props, int input) {
FieldList sourceList;
RequestedGlobalProperties returnProps = null;
RequestedGlobalProperties returnProps = new RequestedGlobalProperties();
// no semantic properties available. All global properties are filtered.
if (props == null) {
return null;
throw new NullPointerException("SemanticProperties may not be null.");
}
// check if partitioning survives
if (this.ordering != null) {
Ordering no = new Ordering();
returnProps = new RequestedGlobalProperties();
returnProps.setPartitioningFields(new FieldSet(), this.partitioning);
RequestedGlobalProperties rgProp = new RequestedGlobalProperties();
for (int index = 0; index < this.ordering.getInvolvedIndexes().size(); index++) {
int value = this.ordering.getInvolvedIndexes().get(index);
sourceList = props.getSourceField(input, value) == null ? null : props.getSourceField(input, value).toFieldList();
if (sourceList != null) {
no.appendOrdering(sourceList.get(0), this.ordering.getType(index), this.ordering.getOrder(index));
} else {
return null;
switch(this.partitioning) {
case FULL_REPLICATION:
case FORCED_REBALANCED:
case CUSTOM_PARTITIONING:
case RANDOM:
// make sure that certain properties are not pushed down
return null;
case HASH_PARTITIONED:
case ANY_PARTITIONING:
FieldSet newFields = new FieldSet();
for (Integer targetField : this.partitioningFields) {
int sourceField = props.getForwardingSourceField(input, targetField);
if (sourceField >= 0) {
newFields = newFields.addField(sourceField);
} else {
// partial partitionings are not preserved to avoid skewed partitioning
return null;
}
}
}
returnProps.setOrdering(no);
} else if (this.partitioningFields != null) {
returnProps = new RequestedGlobalProperties();
returnProps.setPartitioningFields(new FieldSet(), this.partitioning);
for (Integer index : this.partitioningFields) {
sourceList = props.getSourceField(input, index) == null ? null : props.getSourceField(input, index).toFieldList();
if (sourceList != null) {
returnProps.setPartitioningFields(returnProps.getPartitionedFields().addFields(sourceList), this.partitioning);
} else {
return null;
rgProp.partitioning = this.partitioning;
rgProp.partitioningFields = newFields;
return rgProp;
case RANGE_PARTITIONED:
// range partitioning
Ordering newOrdering = new Ordering();
for (int i = 0; i < this.ordering.getInvolvedIndexes().size(); i++) {
int value = this.ordering.getInvolvedIndexes().get(i);
int sourceField = props.getForwardingSourceField(input, value);
if (sourceField >= 0) {
newOrdering.appendOrdering(sourceField, this.ordering.getType(i), this.ordering.getOrder(i));
} else {
return null;
}
}
}
}
// make sure that certain properties are not pushed down
final PartitioningProperty partitioning = this.partitioning;
if (partitioning == PartitioningProperty.FULL_REPLICATION ||
partitioning == PartitioningProperty.FORCED_REBALANCED ||
partitioning == PartitioningProperty.CUSTOM_PARTITIONING)
{
return null;
rgProp.partitioning = this.partitioning;
rgProp.ordering = newOrdering;
rgProp.dataDistribution = this.dataDistribution;
return rgProp;
default:
throw new RuntimeException("Unknown partitioning type encountered.");
}
return returnProps;
}
/**
......
......@@ -23,7 +23,6 @@ import java.util.Arrays;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.compiler.plan.Channel;
import org.apache.flink.compiler.util.Utils;
......@@ -135,50 +134,50 @@ public class RequestedLocalProperties implements Cloneable {
// --------------------------------------------------------------------------------------------
/**
* Filters these properties by what can be preserved through a user function's constant fields set.
* Since interesting properties are filtered top-down, anything that partially destroys the ordering
* makes the properties uninteresting.
*
* @param props The optimizer node that potentially modifies the properties.
* @param input The input of the node which is relevant.
* Filters these properties by what can be preserved by the given SemanticProperties when propagated down
* to the given input.
*
* @param props The SemanticProperties which define which fields are preserved.
* @param input The index of the operator's input.
* @return The filtered RequestedLocalProperties
*/
public RequestedLocalProperties filterBySemanticProperties(SemanticProperties props, int input) {
FieldList sourceList;
RequestedLocalProperties returnProps = this;
// no semantic properties, all local properties are filtered
if (props == null) {
return null;
throw new NullPointerException("SemanticProperties may not be null.");
}
if (this.ordering != null) {
Ordering no = new Ordering();
returnProps = this.clone();
for (int index = 0; index < this.ordering.getInvolvedIndexes().size(); index++) {
int value = this.ordering.getInvolvedIndexes().get(index);
sourceList = props.getSourceField(input, value) == null ? null : props.getSourceField(input, value).toFieldList();
if (sourceList != null) {
no.appendOrdering(sourceList.get(0), this.ordering.getType(index), this.ordering.getOrder(index));
Ordering newOrdering = new Ordering();
for (int i = 0; i < this.ordering.getInvolvedIndexes().size(); i++) {
int targetField = this.ordering.getInvolvedIndexes().get(i);
int sourceField = props.getForwardingSourceField(input, targetField);
if (sourceField >= 0) {
newOrdering.appendOrdering(sourceField, this.ordering.getType(i), this.ordering.getOrder(i));
} else {
return null;
}
}
returnProps.setOrdering(no);
return new RequestedLocalProperties(newOrdering);
} else if (this.groupedFields != null) {
returnProps = this.clone();
returnProps.setGroupedFields(new FieldList());
FieldSet newGrouping = new FieldSet();
// check, whether the local key grouping is preserved
for (Integer index : this.groupedFields) {
sourceList = props.getSourceField(input, index) == null ? null : props.getSourceField(input, index).toFieldList();
if (sourceList != null) {
returnProps.setGroupedFields(returnProps.getGroupedFields().addFields(sourceList));
for (Integer targetField : this.groupedFields) {
int sourceField = props.getForwardingSourceField(input, targetField);
if (sourceField >= 0) {
newGrouping = newGrouping.addField(sourceField);
} else {
return null;
}
}
return new RequestedLocalProperties(newGrouping);
} else {
return null;
}
return returnProps;
}
/**
......
......@@ -525,7 +525,7 @@ public abstract class GenericFlatTypePostPass<X, T extends AbstractSchema<X>> im
Integer pos = entry.getKey();
SemanticProperties sprops = optNode.getSemanticProperties();
if (sprops != null && sprops.getForwardFields(input, pos) != null && sprops.getForwardFields(input,pos).contains(pos)) {
if (sprops != null && sprops.getForwardingTargetFields(input, pos) != null && sprops.getForwardingTargetFields(input, pos).contains(pos)) {
targetSchema.addType(pos, entry.getValue());
}
}
......
......@@ -250,7 +250,7 @@ public class DOPChangeTest extends CompilerTestBase {
// submit the plan to the compiler
OptimizedPlan oPlan = compileNoStats(plan);
// check the optimized Plan
// when reducer 1 distributes its data across the instances of map2, it needs to employ a local hash method,
// because map2 has twice as many instances and key/value pairs with the same key need to be processed by the same
......@@ -258,10 +258,7 @@ public class DOPChangeTest extends CompilerTestBase {
SinkPlanNode sinkNode = oPlan.getDataSinks().iterator().next();
SingleInputPlanNode red2Node = (SingleInputPlanNode) sinkNode.getPredecessor();
SingleInputPlanNode map2Node = (SingleInputPlanNode) red2Node.getPredecessor();
Assert.assertEquals("The Reduce 2 Node has an invalid local strategy.", LocalStrategy.NONE, red2Node.getInput().getLocalStrategy());
Assert.assertEquals("The Map 2 Node has an invalid local strategy.", LocalStrategy.SORT, map2Node.getInput().getLocalStrategy());
Assert.assertEquals("The Reduce 2 Node has an invalid local strategy.", LocalStrategy.SORT, red2Node.getInput().getLocalStrategy());
}
/**
......
......@@ -34,7 +34,7 @@ import org.apache.flink.api.common.functions.RichGroupReduceFunction;
import org.apache.flink.api.common.functions.RichJoinFunction;
import org.apache.flink.api.common.functions.RichMapFunction;
import org.apache.flink.api.common.operators.base.JoinOperatorBase.JoinHint;
import org.apache.flink.api.java.functions.FunctionAnnotation.ConstantFields;
import org.apache.flink.api.java.functions.FunctionAnnotation.ForwardedFields;
import org.apache.flink.api.java.tuple.Tuple1;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.compiler.plan.BulkIterationPlanNode;
......@@ -376,14 +376,14 @@ public class IterationsCompilerTest extends CompilerTestBase {
}
}
@ConstantFields("0")
@ForwardedFields("0")
public static final class Reduce101 extends RichGroupReduceFunction<Tuple1<Long>, Tuple1<Long>> {
@Override
public void reduce(Iterable<Tuple1<Long>> values, Collector<Tuple1<Long>> out) {}
}
@ConstantFields("0")
@ForwardedFields("0")
public static final class DuplicateValue extends RichMapFunction<Tuple1<Long>, Tuple2<Long, Long>> {
@Override
......
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.compiler;
import org.apache.flink.api.common.functions.JoinFunction;
import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.common.functions.ReduceFunction;
import org.apache.flink.api.common.operators.base.JoinOperatorBase;
import org.apache.flink.api.common.operators.base.MapOperatorBase;
import org.apache.flink.api.common.operators.base.ReduceOperatorBase;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.java.DataSet;
import org.apache.flink.api.java.ExecutionEnvironment;
import org.apache.flink.api.java.operators.translation.JavaPlan;
import org.apache.flink.api.java.tuple.Tuple3;
import org.apache.flink.api.java.tuple.Tuple8;
import org.apache.flink.api.java.typeutils.TupleTypeInfo;
import org.apache.flink.compiler.dataproperties.GlobalProperties;
import org.apache.flink.compiler.dataproperties.LocalProperties;
import org.apache.flink.compiler.dataproperties.PartitioningProperty;
import org.apache.flink.compiler.plan.Channel;
import org.apache.flink.compiler.plan.DualInputPlanNode;
import org.apache.flink.compiler.plan.OptimizedPlan;
import org.apache.flink.compiler.plan.PlanNode;
import org.apache.flink.compiler.plan.SingleInputPlanNode;
import org.apache.flink.runtime.operators.shipping.ShipStrategyType;
import org.apache.flink.util.Visitor;
import org.junit.Assert;
import org.junit.Test;
public class SemanticPropertiesAPIToPlanTest extends CompilerTestBase {
private TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>> tupleInfo =
new TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>>(
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO,
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO
);
@Test
public void forwardFieldsTestMapReduce() {
ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
DataSet<Tuple3<Integer, Integer, Integer>> set = env.readCsvFile(IN_FILE).types(Integer.class, Integer.class, Integer.class);
set = set.map(new MockMapper()).withForwardedFields("*")
.groupBy(0)
.reduce(new MockReducer()).withForwardedFields("f0->f1")
.map(new MockMapper()).withForwardedFields("*")
.groupBy(1)
.reduce(new MockReducer()).withForwardedFields("*");
set.print();
JavaPlan plan = env.createProgramPlan();
OptimizedPlan oPlan = compileWithStats(plan);
oPlan.accept(new Visitor<PlanNode>() {
@Override
public boolean preVisit(PlanNode visitable) {
if (visitable instanceof SingleInputPlanNode && visitable.getPactContract() instanceof ReduceOperatorBase) {
for (Channel input: visitable.getInputs()) {
GlobalProperties gprops = visitable.getGlobalProperties();
LocalProperties lprops = visitable.getLocalProperties();
Assert.assertTrue("Reduce should just forward the input if it is already partitioned",
input.getShipStrategy() == ShipStrategyType.FORWARD);
Assert.assertTrue("Wrong GlobalProperties on Reducer",
gprops.isPartitionedOnFields(new FieldSet(1)));
Assert.assertTrue("Wrong GlobalProperties on Reducer",
gprops.getPartitioning() == PartitioningProperty.HASH_PARTITIONED);
Assert.assertTrue("Wrong LocalProperties on Reducer",
lprops.getGroupedFields().contains(1));
}
}
if (visitable instanceof SingleInputPlanNode && visitable.getPactContract() instanceof MapOperatorBase) {
for (Channel input: visitable.getInputs()) {
GlobalProperties gprops = visitable.getGlobalProperties();
LocalProperties lprops = visitable.getLocalProperties();
Assert.assertTrue("Map should just forward the input if it is already partitioned",
input.getShipStrategy() == ShipStrategyType.FORWARD);
Assert.assertTrue("Wrong GlobalProperties on Mapper",
gprops.isPartitionedOnFields(new FieldSet(1)));
Assert.assertTrue("Wrong GlobalProperties on Mapper",
gprops.getPartitioning() == PartitioningProperty.HASH_PARTITIONED);
Assert.assertTrue("Wrong LocalProperties on Mapper",
lprops.getGroupedFields().contains(1));
}
return false;
}
return true;
}
@Override
public void postVisit(PlanNode visitable) {
}
});
}
@Test
public void forwardFieldsTestJoin() {
ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
DataSet<Tuple3<Integer, Integer, Integer>> in1 = env.readCsvFile(IN_FILE).types(Integer.class, Integer.class, Integer.class);
DataSet<Tuple3<Integer, Integer, Integer>> in2 = env.readCsvFile(IN_FILE).types(Integer.class, Integer.class, Integer.class);
in1 = in1.map(new MockMapper()).withForwardedFields("*")
.groupBy(0)
.reduce(new MockReducer()).withForwardedFields("f0->f1");
in2 = in2.map(new MockMapper()).withForwardedFields("*")
.groupBy(1)
.reduce(new MockReducer()).withForwardedFields("f1->f2");
DataSet<Tuple3<Integer, Integer, Integer>> out = in1.join(in2).where(1).equalTo(2).with(new MockJoin());
out.print();
JavaPlan plan = env.createProgramPlan();
OptimizedPlan oPlan = compileWithStats(plan);
oPlan.accept(new Visitor<PlanNode>() {
@Override
public boolean preVisit(PlanNode visitable) {
if (visitable instanceof DualInputPlanNode && visitable.getPactContract() instanceof JoinOperatorBase) {
DualInputPlanNode node = ((DualInputPlanNode) visitable);
final Channel inConn1 = node.getInput1();
final Channel inConn2 = node.getInput2();
Assert.assertTrue("Join should just forward the input if it is already partitioned",
inConn1.getShipStrategy() == ShipStrategyType.FORWARD);
Assert.assertTrue("Join should just forward the input if it is already partitioned",
inConn2.getShipStrategy() == ShipStrategyType.FORWARD);
return false;
}
return true;
}
@Override
public void postVisit(PlanNode visitable) {
}
});
}
public static class MockMapper implements MapFunction<Tuple3<Integer, Integer, Integer>, Tuple3<Integer, Integer, Integer>> {
@Override
public Tuple3<Integer, Integer, Integer> map(Tuple3<Integer, Integer, Integer> value) throws Exception {
return null;
}
}
public static class MockReducer implements ReduceFunction<Tuple3<Integer, Integer, Integer>> {
@Override
public Tuple3<Integer, Integer, Integer> reduce(Tuple3<Integer, Integer, Integer> value1, Tuple3<Integer, Integer, Integer> value2) throws Exception {
return null;
}
}
public static class MockJoin implements JoinFunction<Tuple3<Integer, Integer, Integer>,
Tuple3<Integer, Integer, Integer>, Tuple3<Integer, Integer, Integer>> {
@Override
public Tuple3<Integer, Integer, Integer> join(Tuple3<Integer, Integer, Integer> first, Tuple3<Integer, Integer, Integer> second) throws Exception {
return null;
}
}
}
......@@ -47,10 +47,10 @@ public class SortPartialReuseTest extends CompilerTestBase {
input
.partitionByHash(0)
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1", "2")
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1", "2")
.groupBy(0, 1)
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1", "2")
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1", "2")
.groupBy(0)
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>())
......@@ -92,10 +92,10 @@ public class SortPartialReuseTest extends CompilerTestBase {
@Override
public int partition(Long key, int numPartitions) { return 0; }
}, 0)
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1", "2")
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1", "2")
.groupBy(0, 1)
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1", "2")
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1", "2")
.groupBy(1)
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>())
......
......@@ -239,14 +239,14 @@ public class CoGroupCustomPartitioningTest extends CompilerTestBase {
@Override
public int partition(Long key, int numPartitions) { return 0; }
}, 0)
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1", "2");
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1", "2");
DataSet<Tuple3<Long, Long, Long>> grouped = partitioned
.distinct(0, 1)
.groupBy(1)
.sortGroup(0, Order.ASCENDING)
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1");
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1");
grouped
.coGroup(partitioned).where(0).equalTo(0)
......
......@@ -236,14 +236,14 @@ public class JoinCustomPartitioningTest extends CompilerTestBase {
@Override
public int partition(Long key, int numPartitions) { return 0; }
}, 0)
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1", "2");
.map(new IdentityMapper<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1", "2");
DataSet<Tuple3<Long, Long, Long>> grouped = partitioned
.distinct(0, 1)
.groupBy(1)
.sortGroup(0, Order.ASCENDING)
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withConstantSet("0", "1");
.reduceGroup(new IdentityGroupReducer<Tuple3<Long,Long,Long>>()).withForwardedFields("0", "1");
grouped
.join(partitioned, JoinHint.REPARTITION_HASH_FIRST).where(0).equalTo(0)
......
......@@ -22,34 +22,408 @@ import static org.junit.Assert.*;
import static org.mockito.Mockito.*;
import org.apache.flink.api.common.functions.Partitioner;
import org.apache.flink.api.common.operators.Order;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SingleInputSemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.compiler.dag.OptimizerNode;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.java.functions.SemanticPropUtil;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.api.java.tuple.Tuple8;
import org.apache.flink.api.java.typeutils.TupleTypeInfo;
import org.apache.flink.types.IntValue;
import org.apache.flink.types.LongValue;
import org.apache.flink.types.StringValue;
import org.junit.Assert;
import org.junit.Test;
import org.mockito.Matchers;
import java.util.Set;
public class GlobalPropertiesFilteringTest {
private TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>> tupleInfo =
new TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>>(
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO,
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO
);
@Test
public void testAllErased1() {
SingleInputSemanticProperties semProps = new SingleInputSemanticProperties();
GlobalProperties gprops = new GlobalProperties();
gprops.setHashPartitioned(new FieldList(0, 1));
gprops.addUniqueFieldCombination(new FieldSet(3, 4));
gprops.addUniqueFieldCombination(new FieldSet(5, 6));
GlobalProperties result = gprops.filterBySemanticProperties(semProps, 0);
assertEquals(PartitioningProperty.RANDOM, result.getPartitioning());
assertNull(result.getPartitioningFields());
assertNull(result.getPartitioningOrdering());
assertNull(result.getUniqueFieldCombination());
}
@Test
public void testAllErased2() {
SingleInputSemanticProperties semProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(semProps, new String[]{"2"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setHashPartitioned(new FieldList(0, 1));
gprops.addUniqueFieldCombination(new FieldSet(3, 4));
gprops.addUniqueFieldCombination(new FieldSet(5, 6));
GlobalProperties result = gprops.filterBySemanticProperties(semProps, 0);
assertEquals(PartitioningProperty.RANDOM, result.getPartitioning());
assertNull(result.getPartitioningFields());
assertNull(result.getPartitioningOrdering());
assertNull(result.getUniqueFieldCombination());
}
@Test
public void testHashPartitioningPreserved1() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1;4"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setHashPartitioned(new FieldList(0, 1, 4));
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.HASH_PARTITIONED, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(3, pFields.size());
assertTrue(pFields.contains(0));
assertTrue(pFields.contains(1));
assertTrue(pFields.contains(4));
}
@Test
public void testHashPartitioningPreserved2() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0->1; 1->2; 4->3"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setHashPartitioned(new FieldList(0, 1, 4));
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.HASH_PARTITIONED, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(3, pFields.size());
assertTrue(pFields.contains(1));
assertTrue(pFields.contains(2));
assertTrue(pFields.contains(3));
}
@Test
public void testHashPartitioningErased() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setHashPartitioned(new FieldList(0, 1, 4));
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.RANDOM, result.getPartitioning());
assertNull(result.getPartitioningFields());
}
@Test
public void testAnyPartitioningPreserved1() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1;4"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setAnyPartitioning(new FieldList(0, 1, 4));
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.ANY_PARTITIONING, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(3, pFields.size());
assertTrue(pFields.contains(0));
assertTrue(pFields.contains(1));
assertTrue(pFields.contains(4));
}
@Test
public void testAnyPartitioningPreserved2() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0->1; 1->2; 4->3"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setAnyPartitioning(new FieldList(0, 1, 4));
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.ANY_PARTITIONING, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(3, pFields.size());
assertTrue(pFields.contains(1));
assertTrue(pFields.contains(2));
assertTrue(pFields.contains(3));
}
@Test
public void testAnyPartitioningErased() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setAnyPartitioning(new FieldList(0, 1, 4));
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.RANDOM, result.getPartitioning());
assertNull(result.getPartitioningFields());
}
@Test
public void testCustomPartitioningPreserves() {
try {
Partitioner<?> partitioner = new MockPartitioner();
GlobalProperties gp = new GlobalProperties();
gp.setCustomPartitioned(new FieldList(2, 3), partitioner);
OptimizerNode node = mock(OptimizerNode.class);
when(node.isFieldConstant(Matchers.anyInt(), Matchers.anyInt())).thenReturn(true);
GlobalProperties filtered = gp.filterByNodesConstantSet(node, 0);
assertTrue(filtered.isPartitionedOnFields(new FieldSet(2, 3)));
assertEquals(PartitioningProperty.CUSTOM_PARTITIONING, filtered.getPartitioning());
assertEquals(partitioner, filtered.getCustomPartitioner());
}
catch (Exception e) {
e.printStackTrace();
fail(e.getMessage());
}
public void testCustomPartitioningPreserved1() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1;4"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
Partitioner<Tuple2<Long, Integer>> myP = new MockPartitioner();
gprops.setCustomPartitioned(new FieldList(0, 4), myP);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.CUSTOM_PARTITIONING, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(2, pFields.size());
assertTrue(pFields.contains(0));
assertTrue(pFields.contains(4));
assertEquals(myP, result.getCustomPartitioner());
}
@Test
public void testCustomPartitioningPreserved2() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0->1; 1->2; 4->3"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
Partitioner<Tuple2<Long, Integer>> myP = new MockPartitioner();
gprops.setCustomPartitioned(new FieldList(0, 4), myP);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.CUSTOM_PARTITIONING, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(2, pFields.size());
assertTrue(pFields.contains(1));
assertTrue(pFields.contains(3));
assertEquals(myP, result.getCustomPartitioner());
}
@Test
public void testCustomPartitioningErased() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
Partitioner<Tuple2<Long, Integer>> myP = new MockPartitioner();
gprops.setCustomPartitioned(new FieldList(0, 4), myP);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.RANDOM, result.getPartitioning());
assertNull(result.getPartitioningFields());
assertNull(result.getCustomPartitioner());
}
@Test
public void testRangePartitioningPreserved1() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"1;2;5"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
o.appendOrdering(2, StringValue.class, Order.ASCENDING);
GlobalProperties gprops = new GlobalProperties();
gprops.setRangePartitioned(o);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.RANGE_PARTITIONED, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(3, pFields.size());
assertEquals(1, pFields.get(0).intValue());
assertEquals(5, pFields.get(1).intValue());
assertEquals(2, pFields.get(2).intValue());
Ordering pOrder = result.getPartitioningOrdering();
assertEquals(3, pOrder.getNumberOfFields());
assertEquals(1, pOrder.getFieldNumber(0).intValue());
assertEquals(5, pOrder.getFieldNumber(1).intValue());
assertEquals(2, pOrder.getFieldNumber(2).intValue());
assertEquals(Order.ASCENDING, pOrder.getOrder(0));
assertEquals(Order.DESCENDING, pOrder.getOrder(1));
assertEquals(Order.ASCENDING, pOrder.getOrder(2));
assertEquals(IntValue.class, pOrder.getType(0));
assertEquals(LongValue.class, pOrder.getType(1));
assertEquals(StringValue.class, pOrder.getType(2));
}
@Test
public void testRangePartitioningPreserved2() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"1->3; 2->0; 5->1"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
o.appendOrdering(2, StringValue.class, Order.ASCENDING);
GlobalProperties gprops = new GlobalProperties();
gprops.setRangePartitioned(o);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.RANGE_PARTITIONED, result.getPartitioning());
FieldList pFields = result.getPartitioningFields();
assertEquals(3, pFields.size());
assertEquals(3, pFields.get(0).intValue());
assertEquals(1, pFields.get(1).intValue());
assertEquals(0, pFields.get(2).intValue());
Ordering pOrder = result.getPartitioningOrdering();
assertEquals(3, pOrder.getNumberOfFields());
assertEquals(3, pOrder.getFieldNumber(0).intValue());
assertEquals(1, pOrder.getFieldNumber(1).intValue());
assertEquals(0, pOrder.getFieldNumber(2).intValue());
assertEquals(Order.ASCENDING, pOrder.getOrder(0));
assertEquals(Order.DESCENDING, pOrder.getOrder(1));
assertEquals(Order.ASCENDING, pOrder.getOrder(2));
assertEquals(IntValue.class, pOrder.getType(0));
assertEquals(LongValue.class, pOrder.getType(1));
assertEquals(StringValue.class, pOrder.getType(2));
}
@Test
public void testRangePartitioningErased() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"1;5"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
o.appendOrdering(2, StringValue.class, Order.ASCENDING);
GlobalProperties gprops = new GlobalProperties();
gprops.setRangePartitioned(o);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.RANDOM, result.getPartitioning());
assertNull(result.getPartitioningOrdering());
assertNull(result.getPartitioningFields());
}
@Test
public void testRebalancingPreserved() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0->1; 1->2; 4->3"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setForcedRebalanced();
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
assertEquals(PartitioningProperty.FORCED_REBALANCED, result.getPartitioning());
assertNull(result.getPartitioningFields());
}
@Test
public void testUniqueFieldGroupsPreserved1() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1;2;3;4"}, null, null, tupleInfo, tupleInfo);
FieldSet set1 = new FieldSet(0, 1, 2);
FieldSet set2 = new FieldSet(3, 4);
FieldSet set3 = new FieldSet(4, 5, 6, 7);
GlobalProperties gprops = new GlobalProperties();
gprops.addUniqueFieldCombination(set1);
gprops.addUniqueFieldCombination(set2);
gprops.addUniqueFieldCombination(set3);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
Set<FieldSet> unique = result.getUniqueFieldCombination();
FieldSet expected1 = new FieldSet(0, 1, 2);
FieldSet expected2 = new FieldSet(3, 4);
Assert.assertTrue(unique.size() == 2);
Assert.assertTrue(unique.contains(expected1));
Assert.assertTrue(unique.contains(expected2));
}
@Test
public void testUniqueFieldGroupsPreserved2() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0->5;1;2;3->6;4"}, null, null, tupleInfo, tupleInfo);
FieldSet set1 = new FieldSet(0, 1, 2);
FieldSet set2 = new FieldSet(3, 4);
FieldSet set3 = new FieldSet(4, 5, 6, 7);
GlobalProperties gprops = new GlobalProperties();
gprops.addUniqueFieldCombination(set1);
gprops.addUniqueFieldCombination(set2);
gprops.addUniqueFieldCombination(set3);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
Set<FieldSet> unique = result.getUniqueFieldCombination();
FieldSet expected1 = new FieldSet(1, 2, 5);
FieldSet expected2 = new FieldSet(4, 6);
Assert.assertTrue(unique.size() == 2);
Assert.assertTrue(unique.contains(expected1));
Assert.assertTrue(unique.contains(expected2));
}
@Test
public void testUniqueFieldGroupsErased() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0; 3; 5; 6; 7"}, null, null, tupleInfo, tupleInfo);
FieldSet set1 = new FieldSet(0, 1, 2);
FieldSet set2 = new FieldSet(3, 4);
FieldSet set3 = new FieldSet(4, 5, 6, 7);
GlobalProperties gprops = new GlobalProperties();
gprops.addUniqueFieldCombination(set1);
gprops.addUniqueFieldCombination(set2);
gprops.addUniqueFieldCombination(set3);
GlobalProperties result = gprops.filterBySemanticProperties(sprops, 0);
Assert.assertNull(result.getUniqueFieldCombination());
}
@Test(expected = IndexOutOfBoundsException.class)
public void testInvalidInputIndex() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1"}, null, null, tupleInfo, tupleInfo);
GlobalProperties gprops = new GlobalProperties();
gprops.setHashPartitioned(new FieldList(0, 1));
gprops.filterBySemanticProperties(sprops, 1);
}
}
......@@ -25,6 +25,7 @@ import org.apache.flink.api.common.operators.Order;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.api.java.tuple.Tuple2;
import org.junit.Test;
public class GlobalPropertiesMatchingTest {
......@@ -113,7 +114,7 @@ public class GlobalPropertiesMatchingTest {
@Test
public void testMatchingCustomPartitioning() {
try {
final Partitioner<Long> partitioner = new MockPartitioner();
final Partitioner<Tuple2<Long, Integer>> partitioner = new MockPartitioner();
RequestedGlobalProperties req = new RequestedGlobalProperties();
req.setCustomPartitioned(new FieldSet(6, 2), partitioner);
......
......@@ -20,12 +20,11 @@ package org.apache.flink.compiler.dataproperties;
import static org.junit.Assert.*;
import static org.mockito.Mockito.mock;
import static org.mockito.Mockito.when;
import org.apache.flink.api.common.operators.SemanticProperties;
import org.apache.flink.api.common.operators.SingleInputSemanticProperties;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.compiler.dag.OptimizerNode;
import org.junit.Test;
import org.mockito.Matchers;
public class GlobalPropertiesPushdownTest {
......@@ -35,11 +34,11 @@ public class GlobalPropertiesPushdownTest {
RequestedGlobalProperties req = new RequestedGlobalProperties();
req.setAnyPartitioning(new FieldSet(3, 1));
RequestedGlobalProperties preserved = req.filterByNodesConstantSet(getAllPreservingNode(), 0);
RequestedGlobalProperties preserved = req.filterBySemanticProperties(getAllPreservingSemProps(), 0);
assertEquals(PartitioningProperty.ANY_PARTITIONING, preserved.getPartitioning());
assertTrue(preserved.getPartitionedFields().isValidSubset(new FieldSet(1, 3)));
RequestedGlobalProperties nonPreserved = req.filterByNodesConstantSet(getNonePreservingNode(), 0);
RequestedGlobalProperties nonPreserved = req.filterBySemanticProperties(getNonePreservingSemProps(), 0);
assertTrue(nonPreserved == null || nonPreserved.isTrivial());
}
catch (Exception e) {
......@@ -54,11 +53,11 @@ public class GlobalPropertiesPushdownTest {
RequestedGlobalProperties req = new RequestedGlobalProperties();
req.setHashPartitioned(new FieldSet(3, 1));
RequestedGlobalProperties preserved = req.filterByNodesConstantSet(getAllPreservingNode(), 0);
RequestedGlobalProperties preserved = req.filterBySemanticProperties(getAllPreservingSemProps(), 0);
assertEquals(PartitioningProperty.HASH_PARTITIONED, preserved.getPartitioning());
assertTrue(preserved.getPartitionedFields().isValidSubset(new FieldSet(1, 3)));
RequestedGlobalProperties nonPreserved = req.filterByNodesConstantSet(getNonePreservingNode(), 0);
RequestedGlobalProperties nonPreserved = req.filterBySemanticProperties(getNonePreservingSemProps(), 0);
assertTrue(nonPreserved == null || nonPreserved.isTrivial());
}
catch (Exception e) {
......@@ -73,7 +72,7 @@ public class GlobalPropertiesPushdownTest {
RequestedGlobalProperties req = new RequestedGlobalProperties();
req.setCustomPartitioned(new FieldSet(3, 1), new MockPartitioner());
RequestedGlobalProperties pushedDown = req.filterByNodesConstantSet(getAllPreservingNode(), 0);
RequestedGlobalProperties pushedDown = req.filterBySemanticProperties(getAllPreservingSemProps(), 0);
assertTrue(pushedDown == null || pushedDown.isTrivial());
}
catch (Exception e) {
......@@ -88,7 +87,7 @@ public class GlobalPropertiesPushdownTest {
RequestedGlobalProperties req = new RequestedGlobalProperties();
req.setForceRebalancing();
RequestedGlobalProperties pushedDown = req.filterByNodesConstantSet(getAllPreservingNode(), 0);
RequestedGlobalProperties pushedDown = req.filterBySemanticProperties(getAllPreservingSemProps(), 0);
assertTrue(pushedDown == null || pushedDown.isTrivial());
}
catch (Exception e) {
......@@ -99,15 +98,11 @@ public class GlobalPropertiesPushdownTest {
// --------------------------------------------------------------------------------------------
private static OptimizerNode getAllPreservingNode() {
OptimizerNode node = mock(OptimizerNode.class);
when(node.isFieldConstant(Matchers.anyInt(), Matchers.anyInt())).thenReturn(true);
return node;
private static SemanticProperties getAllPreservingSemProps() {
return new SingleInputSemanticProperties.AllFieldsForwardedProperties();
}
private static OptimizerNode getNonePreservingNode() {
OptimizerNode node = mock(OptimizerNode.class);
when(node.isFieldConstant(Matchers.anyInt(), Matchers.anyInt())).thenReturn(false);
return node;
private static SemanticProperties getNonePreservingSemProps() {
return new SingleInputSemanticProperties();
}
}
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.compiler.dataproperties;
import static org.junit.Assert.*;
import org.apache.flink.api.common.operators.Order;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SingleInputSemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.java.functions.SemanticPropUtil;
import org.apache.flink.api.java.tuple.Tuple8;
import org.apache.flink.api.java.typeutils.TupleTypeInfo;
import org.apache.flink.types.IntValue;
import org.apache.flink.types.LongValue;
import org.apache.flink.types.StringValue;
import org.junit.Assert;
import org.junit.Test;
import java.util.Set;
public class LocalPropertiesFilteringTest {
private TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>> tupleInfo =
new TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>>(
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO,
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO
);
@Test
public void testAllErased1() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, null, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = LocalProperties.forGrouping(new FieldList(0, 1, 2));
lProps = lProps.addUniqueFields(new FieldSet(3,4));
lProps = lProps.addUniqueFields(new FieldSet(5,6));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
assertNull(filtered.getGroupedFields());
assertNull(filtered.getOrdering());
assertNull(filtered.getUniqueFields());
}
@Test
public void testAllErased2() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"5"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = LocalProperties.forGrouping(new FieldList(0, 1, 2));
lProps = lProps.addUniqueFields(new FieldSet(3,4));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
assertNull(filtered.getGroupedFields());
assertNull(filtered.getOrdering());
assertNull(filtered.getUniqueFields());
}
@Test
public void testGroupingPreserved1() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;2;3"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = LocalProperties.forGrouping(new FieldList(0, 2, 3));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
assertNotNull(filtered.getGroupedFields());
assertEquals(3, filtered.getGroupedFields().size());
assertTrue(filtered.getGroupedFields().contains(0));
assertTrue(filtered.getGroupedFields().contains(2));
assertTrue(filtered.getGroupedFields().contains(3));
assertNull(filtered.getOrdering());
assertNull(filtered.getUniqueFields());
}
@Test
public void testGroupingPreserved2() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0->4;2->0;3->7"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = LocalProperties.forGrouping(new FieldList(0, 2, 3));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
assertNotNull(filtered.getGroupedFields());
assertEquals(3, filtered.getGroupedFields().size());
assertTrue(filtered.getGroupedFields().contains(4));
assertTrue(filtered.getGroupedFields().contains(0));
assertTrue(filtered.getGroupedFields().contains(7));
assertNull(filtered.getOrdering());
assertNull(filtered.getUniqueFields());
}
@Test
public void testGroupingErased() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0->4;2->0"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = LocalProperties.forGrouping(new FieldList(0, 2, 3));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
assertNull(filtered.getGroupedFields());
assertNull(filtered.getOrdering());
assertNull(filtered.getUniqueFields());
}
@Test
public void testSortingPreserved1() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;2;5"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(2, IntValue.class, Order.ASCENDING);
o.appendOrdering(0, StringValue.class, Order.DESCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
LocalProperties lProps = LocalProperties.forOrdering(o);
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldList gFields = filtered.getGroupedFields();
Ordering order = filtered.getOrdering();
assertNotNull(gFields);
assertEquals(3, gFields.size());
assertTrue(gFields.contains(0));
assertTrue(gFields.contains(2));
assertTrue(gFields.contains(5));
assertNotNull(order);
assertEquals(3, order.getNumberOfFields());
assertEquals(2, order.getFieldNumber(0).intValue());
assertEquals(0, order.getFieldNumber(1).intValue());
assertEquals(5, order.getFieldNumber(2).intValue());
assertEquals(Order.ASCENDING, order.getOrder(0));
assertEquals(Order.DESCENDING, order.getOrder(1));
assertEquals(Order.DESCENDING, order.getOrder(2));
assertEquals(IntValue.class, order.getType(0));
assertEquals(StringValue.class, order.getType(1));
assertEquals(LongValue.class, order.getType(2));
assertNull(filtered.getUniqueFields());
}
@Test
public void testSortingPreserved2() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0->3;2->7;5->1"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(2, IntValue.class, Order.ASCENDING);
o.appendOrdering(0, StringValue.class, Order.DESCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
LocalProperties lProps = LocalProperties.forOrdering(o);
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldList gFields = filtered.getGroupedFields();
Ordering order = filtered.getOrdering();
assertNotNull(gFields);
assertEquals(3, gFields.size());
assertTrue(gFields.contains(3));
assertTrue(gFields.contains(7));
assertTrue(gFields.contains(1));
assertNotNull(order);
assertEquals(3, order.getNumberOfFields());
assertEquals(7, order.getFieldNumber(0).intValue());
assertEquals(3, order.getFieldNumber(1).intValue());
assertEquals(1, order.getFieldNumber(2).intValue());
assertEquals(Order.ASCENDING, order.getOrder(0));
assertEquals(Order.DESCENDING, order.getOrder(1));
assertEquals(Order.DESCENDING, order.getOrder(2));
assertEquals(IntValue.class, order.getType(0));
assertEquals(StringValue.class, order.getType(1));
assertEquals(LongValue.class, order.getType(2));
assertNull(filtered.getUniqueFields());
}
@Test
public void testSortingPreserved3() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;2"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(2, IntValue.class, Order.ASCENDING);
o.appendOrdering(0, StringValue.class, Order.DESCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
LocalProperties lProps = LocalProperties.forOrdering(o);
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldList gFields = filtered.getGroupedFields();
Ordering order = filtered.getOrdering();
assertNotNull(gFields);
assertEquals(2, gFields.size());
assertTrue(gFields.contains(0));
assertTrue(gFields.contains(2));
assertNotNull(order);
assertEquals(2, order.getNumberOfFields());
assertEquals(2, order.getFieldNumber(0).intValue());
assertEquals(0, order.getFieldNumber(1).intValue());
assertEquals(Order.ASCENDING, order.getOrder(0));
assertEquals(Order.DESCENDING, order.getOrder(1));
assertEquals(IntValue.class, order.getType(0));
assertEquals(StringValue.class, order.getType(1));
assertNull(filtered.getUniqueFields());
}
@Test
public void testSortingPreserved4() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"2->7;5"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(2, IntValue.class, Order.ASCENDING);
o.appendOrdering(0, StringValue.class, Order.DESCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
LocalProperties lProps = LocalProperties.forOrdering(o);
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldList gFields = filtered.getGroupedFields();
Ordering order = filtered.getOrdering();
assertNotNull(gFields);
assertEquals(1, gFields.size());
assertTrue(gFields.contains(7));
assertNotNull(order);
assertEquals(1, order.getNumberOfFields());
assertEquals(7, order.getFieldNumber(0).intValue());
assertEquals(Order.ASCENDING, order.getOrder(0));
assertEquals(IntValue.class, order.getType(0));
assertNull(filtered.getUniqueFields());
}
@Test
public void testSortingErased() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;5"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(2, IntValue.class, Order.ASCENDING);
o.appendOrdering(0, StringValue.class, Order.DESCENDING);
o.appendOrdering(5, LongValue.class, Order.DESCENDING);
LocalProperties lProps = LocalProperties.forOrdering(o);
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldList gFields = filtered.getGroupedFields();
Ordering order = filtered.getOrdering();
assertNull(gFields);
assertNull(order);
assertNull(filtered.getUniqueFields());
}
@Test
public void testUniqueFieldsPreserved1() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;1;2;3;4"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = new LocalProperties();
lProps = lProps.addUniqueFields(new FieldSet(0,1,2));
lProps = lProps.addUniqueFields(new FieldSet(3,4));
lProps = lProps.addUniqueFields(new FieldSet(4,5,6));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldSet expected1 = new FieldSet(0,1,2);
FieldSet expected2 = new FieldSet(3,4);
assertNull(filtered.getGroupedFields());
assertNull(filtered.getOrdering());
assertNotNull(filtered.getUniqueFields());
assertEquals(2, filtered.getUniqueFields().size());
assertTrue(filtered.getUniqueFields().contains(expected1));
assertTrue(filtered.getUniqueFields().contains(expected2));
}
@Test
public void testUniqueFieldsPreserved2() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;1;2;3;4"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = LocalProperties.forGrouping(new FieldList(1,2));
lProps = lProps.addUniqueFields(new FieldSet(0,1,2));
lProps = lProps.addUniqueFields(new FieldSet(3,4));
lProps = lProps.addUniqueFields(new FieldSet(4,5,6));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldSet expected1 = new FieldSet(0,1,2);
FieldSet expected2 = new FieldSet(3,4);
assertNull(filtered.getOrdering());
assertNotNull(filtered.getGroupedFields());
assertEquals(2, filtered.getGroupedFields().size());
assertTrue(filtered.getGroupedFields().contains(1));
assertTrue(filtered.getGroupedFields().contains(2));
assertNotNull(filtered.getUniqueFields());
assertEquals(2, filtered.getUniqueFields().size());
assertTrue(filtered.getUniqueFields().contains(expected1));
assertTrue(filtered.getUniqueFields().contains(expected2));
}
@Test
public void testUniqueFieldsPreserved3() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0->7;1->6;2->5;3->4;4->3"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = new LocalProperties();
lProps = lProps.addUniqueFields(new FieldSet(0,1,2));
lProps = lProps.addUniqueFields(new FieldSet(3,4));
lProps = lProps.addUniqueFields(new FieldSet(4,5,6));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
FieldSet expected1 = new FieldSet(5,6,7);
FieldSet expected2 = new FieldSet(3,4);
assertNull(filtered.getGroupedFields());
assertNull(filtered.getOrdering());
assertNotNull(filtered.getUniqueFields());
assertEquals(2, filtered.getUniqueFields().size());
assertTrue(filtered.getUniqueFields().contains(expected1));
assertTrue(filtered.getUniqueFields().contains(expected2));
}
@Test
public void testUniqueFieldsErased() {
SingleInputSemanticProperties sp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sp, new String[]{"0;1;4"}, null, null, tupleInfo, tupleInfo);
LocalProperties lProps = new LocalProperties();
lProps = lProps.addUniqueFields(new FieldSet(0,1,2));
lProps = lProps.addUniqueFields(new FieldSet(3,4));
lProps = lProps.addUniqueFields(new FieldSet(4,5,6));
LocalProperties filtered = lProps.filterBySemanticProperties(sp, 0);
assertNull(filtered.getGroupedFields());
assertNull(filtered.getOrdering());
assertNull(filtered.getUniqueFields());
}
@Test(expected = IndexOutOfBoundsException.class)
public void testInvalidInputIndex() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1"}, null, null, tupleInfo, tupleInfo);
LocalProperties lprops = LocalProperties.forGrouping(new FieldList(0,1));
lprops.filterBySemanticProperties(sprops, 1);
}
}
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.compiler.dataproperties;
import org.apache.flink.api.common.distributions.DataDistribution;
import org.apache.flink.core.memory.DataInputView;
import org.apache.flink.core.memory.DataOutputView;
import org.apache.flink.types.Key;
import java.io.IOException;
public class MockDistribution implements DataDistribution {
@Override
public Key<?>[] getBucketBoundary(int bucketNum, int totalNumBuckets) {
return new Key<?>[0];
}
@Override
public int getNumberOfFields() {
return 0;
}
@Override
public void write(DataOutputView out) throws IOException {
}
@Override
public void read(DataInputView in) throws IOException {
}
}
......@@ -19,13 +19,14 @@
package org.apache.flink.compiler.dataproperties;
import org.apache.flink.api.common.functions.Partitioner;
import org.apache.flink.api.java.tuple.Tuple2;
class MockPartitioner implements Partitioner<Long> {
class MockPartitioner implements Partitioner<Tuple2<Long, Integer>> {
private static final long serialVersionUID = 1L;
@Override
public int partition(Long key, int numPartitions) {
public int partition(Tuple2<Long, Integer> key, int numPartitions) {
return 0;
}
}
\ No newline at end of file
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.compiler.dataproperties;
import static org.junit.Assert.*;
import org.apache.flink.api.common.distributions.DataDistribution;
import org.apache.flink.api.common.operators.DualInputSemanticProperties;
import org.apache.flink.api.common.operators.Order;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SingleInputSemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.java.functions.SemanticPropUtil;
import org.apache.flink.api.java.tuple.Tuple8;
import org.apache.flink.api.java.typeutils.TupleTypeInfo;
import org.apache.flink.types.ByteValue;
import org.apache.flink.types.IntValue;
import org.apache.flink.types.LongValue;
import org.junit.Assert;
import org.junit.Test;
import java.io.IOException;
public class RequestedGlobalPropertiesFilteringTest {
private TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>> tupleInfo =
new TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>>(
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO,
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO
);
@Test(expected = NullPointerException.class)
public void testNullProps() {
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setAnyPartitioning(new FieldSet(0,1,2));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(null, 0);
}
@Test
public void testEraseAll1() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setAnyPartitioning(new FieldSet(0,1,2));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testEraseAll2() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"3;4"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setAnyPartitioning(new FieldSet(0, 1, 2));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testHashPartitioningPreserved1() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"0;3;4"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setHashPartitioned(new FieldSet(0, 3, 4));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.HASH_PARTITIONED, filtered.getPartitioning());
assertNotNull(filtered.getPartitionedFields());
assertEquals(3, filtered.getPartitionedFields().size());
assertTrue(filtered.getPartitionedFields().contains(0));
assertTrue(filtered.getPartitionedFields().contains(3));
assertTrue(filtered.getPartitionedFields().contains(4));
assertNull(filtered.getDataDistribution());
assertNull(filtered.getCustomPartitioner());
assertNull(filtered.getOrdering());
}
@Test
public void testHashPartitioningPreserved2() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"2->0;1->3;7->4"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setHashPartitioned(new FieldSet(0, 3, 4));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.HASH_PARTITIONED, filtered.getPartitioning());
assertNotNull(filtered.getPartitionedFields());
assertEquals(3, filtered.getPartitionedFields().size());
assertTrue(filtered.getPartitionedFields().contains(1));
assertTrue(filtered.getPartitionedFields().contains(2));
assertTrue(filtered.getPartitionedFields().contains(7));
assertNull(filtered.getDataDistribution());
assertNull(filtered.getCustomPartitioner());
assertNull(filtered.getOrdering());
}
@Test
public void testHashPartitioningErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"1;2"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setHashPartitioned(new FieldSet(0, 3, 4));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testAnyPartitioningPreserved1() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"0;3;4"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setAnyPartitioning(new FieldSet(0, 3, 4));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.ANY_PARTITIONING, filtered.getPartitioning());
assertNotNull(filtered.getPartitionedFields());
assertEquals(3, filtered.getPartitionedFields().size());
assertTrue(filtered.getPartitionedFields().contains(0));
assertTrue(filtered.getPartitionedFields().contains(3));
assertTrue(filtered.getPartitionedFields().contains(4));
assertNull(filtered.getDataDistribution());
assertNull(filtered.getCustomPartitioner());
assertNull(filtered.getOrdering());
}
@Test
public void testAnyPartitioningPreserved2() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"2->0;1->3;7->4"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setAnyPartitioning(new FieldSet(0, 3, 4));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.ANY_PARTITIONING, filtered.getPartitioning());
assertNotNull(filtered.getPartitionedFields());
assertEquals(3, filtered.getPartitionedFields().size());
assertTrue(filtered.getPartitionedFields().contains(1));
assertTrue(filtered.getPartitionedFields().contains(2));
assertTrue(filtered.getPartitionedFields().contains(7));
assertNull(filtered.getDataDistribution());
assertNull(filtered.getCustomPartitioner());
assertNull(filtered.getOrdering());
}
@Test
public void testAnyPartitioningErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"1;2"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setAnyPartitioning(new FieldSet(0, 3, 4));
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testRangePartitioningPreserved1() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"1;3;6"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(3, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setRangePartitioned(o);
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.RANGE_PARTITIONED, filtered.getPartitioning());
assertNotNull(filtered.getOrdering());
assertEquals(3, filtered.getOrdering().getNumberOfFields());
assertEquals(3, filtered.getOrdering().getFieldNumber(0).intValue());
assertEquals(1, filtered.getOrdering().getFieldNumber(1).intValue());
assertEquals(6, filtered.getOrdering().getFieldNumber(2).intValue());
assertEquals(LongValue.class, filtered.getOrdering().getType(0));
assertEquals(IntValue.class, filtered.getOrdering().getType(1));
assertEquals(ByteValue.class, filtered.getOrdering().getType(2));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(0));
assertEquals(Order.ASCENDING, filtered.getOrdering().getOrder(1));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(2));
assertNull(filtered.getPartitionedFields());
assertNull(filtered.getDataDistribution());
assertNull(filtered.getCustomPartitioner());
}
@Test
public void testRangePartitioningPreserved2() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"7->3;1->1;2->6"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(3, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setRangePartitioned(o);
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.RANGE_PARTITIONED, filtered.getPartitioning());
assertNotNull(filtered.getOrdering());
assertEquals(3, filtered.getOrdering().getNumberOfFields());
assertEquals(7, filtered.getOrdering().getFieldNumber(0).intValue());
assertEquals(1, filtered.getOrdering().getFieldNumber(1).intValue());
assertEquals(2, filtered.getOrdering().getFieldNumber(2).intValue());
assertEquals(LongValue.class, filtered.getOrdering().getType(0));
assertEquals(IntValue.class, filtered.getOrdering().getType(1));
assertEquals(ByteValue.class, filtered.getOrdering().getType(2));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(0));
assertEquals(Order.ASCENDING, filtered.getOrdering().getOrder(1));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(2));
assertNull(filtered.getPartitionedFields());
assertNull(filtered.getDataDistribution());
assertNull(filtered.getCustomPartitioner());
}
@Test
public void testRangePartitioningPreserved3() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"7->3;1->1;2->6"}, null, null, tupleInfo, tupleInfo);
DataDistribution dd = new MockDistribution();
Ordering o = new Ordering();
o.appendOrdering(3, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setRangePartitioned(o, dd);
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNotNull(filtered);
assertEquals(PartitioningProperty.RANGE_PARTITIONED, filtered.getPartitioning());
assertNotNull(filtered.getOrdering());
assertEquals(3, filtered.getOrdering().getNumberOfFields());
assertEquals(7, filtered.getOrdering().getFieldNumber(0).intValue());
assertEquals(1, filtered.getOrdering().getFieldNumber(1).intValue());
assertEquals(2, filtered.getOrdering().getFieldNumber(2).intValue());
assertEquals(LongValue.class, filtered.getOrdering().getType(0));
assertEquals(IntValue.class, filtered.getOrdering().getType(1));
assertEquals(ByteValue.class, filtered.getOrdering().getType(2));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(0));
assertEquals(Order.ASCENDING, filtered.getOrdering().getOrder(1));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(2));
assertNotNull(filtered.getDataDistribution());
assertEquals(dd, filtered.getDataDistribution());
assertNull(filtered.getPartitionedFields());
assertNull(filtered.getCustomPartitioner());
}
@Test
public void testRangePartitioningErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"1;2"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(3, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setRangePartitioned(o);
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testCustomPartitioningErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"0;1;2"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setCustomPartitioned(new FieldSet(0, 1, 2), new MockPartitioner());
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testRandomDistributionErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"0;1;2"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setRandomDistribution();
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testReplicationErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"0;1;2"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setFullyReplicated();
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testRebalancingErased() {
SingleInputSemanticProperties sProp = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProp, new String[]{"0;1;2"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties rgProps = new RequestedGlobalProperties();
rgProps.setForceRebalancing();
RequestedGlobalProperties filtered = rgProps.filterBySemanticProperties(sProp, 0);
assertNull(filtered);
}
@Test
public void testDualHashPartitioningPreserved() {
DualInputSemanticProperties dprops = new DualInputSemanticProperties();
SemanticPropUtil.getSemanticPropsDualFromString(dprops, new String[]{"0;2;4"}, new String[]{"1->3;4->6;3->7"},
null, null, null, null, tupleInfo, tupleInfo, tupleInfo);
RequestedGlobalProperties gprops1 = new RequestedGlobalProperties();
RequestedGlobalProperties gprops2 = new RequestedGlobalProperties();
gprops1.setHashPartitioned(new FieldSet(2, 0, 4));
gprops2.setHashPartitioned(new FieldSet(3, 6, 7));
RequestedGlobalProperties filtered1 = gprops1.filterBySemanticProperties(dprops, 0);
RequestedGlobalProperties filtered2 = gprops2.filterBySemanticProperties(dprops, 1);
assertNotNull(filtered1);
assertEquals(PartitioningProperty.HASH_PARTITIONED, filtered1.getPartitioning());
assertNotNull(filtered1.getPartitionedFields());
assertEquals(3, filtered1.getPartitionedFields().size());
assertTrue(filtered1.getPartitionedFields().contains(0));
assertTrue(filtered1.getPartitionedFields().contains(2));
assertTrue(filtered1.getPartitionedFields().contains(4));
assertNull(filtered1.getOrdering());
assertNull(filtered1.getCustomPartitioner());
assertNull(filtered1.getDataDistribution());
assertNotNull(filtered2);
assertEquals(PartitioningProperty.HASH_PARTITIONED, filtered2.getPartitioning());
assertNotNull(filtered2.getPartitionedFields());
assertEquals(3, filtered2.getPartitionedFields().size());
assertTrue(filtered2.getPartitionedFields().contains(1));
assertTrue(filtered2.getPartitionedFields().contains(3));
assertTrue(filtered2.getPartitionedFields().contains(4));
assertNull(filtered2.getOrdering());
assertNull(filtered2.getCustomPartitioner());
assertNull(filtered2.getDataDistribution());
}
@Test(expected = IndexOutOfBoundsException.class)
public void testInvalidInputIndex() {
SingleInputSemanticProperties sprops = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sprops, new String[]{"0;1"}, null, null, tupleInfo, tupleInfo);
RequestedGlobalProperties gprops = new RequestedGlobalProperties();
gprops.setHashPartitioned(new FieldList(0,1));
gprops.filterBySemanticProperties(sprops, 1);
}
}
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.compiler.dataproperties;
import static org.junit.Assert.*;
import org.apache.flink.api.common.operators.DualInputSemanticProperties;
import org.apache.flink.api.common.operators.Order;
import org.apache.flink.api.common.operators.Ordering;
import org.apache.flink.api.common.operators.SingleInputSemanticProperties;
import org.apache.flink.api.common.operators.util.FieldList;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.java.functions.SemanticPropUtil;
import org.apache.flink.api.java.tuple.Tuple8;
import org.apache.flink.api.java.typeutils.TupleTypeInfo;
import org.apache.flink.types.ByteValue;
import org.apache.flink.types.IntValue;
import org.apache.flink.types.LongValue;
import org.junit.Assert;
import org.junit.Test;
public class RequestedLocalPropertiesFilteringTest {
private TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>> tupleInfo =
new TupleTypeInfo<Tuple8<Integer, Integer, Integer, Integer, Integer, Integer, Integer, Integer>>(
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO,
BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.INT_TYPE_INFO
);
@Test(expected = NullPointerException.class)
public void testNullProps() {
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setGroupedFields(new FieldSet(0, 2, 3));
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(null, 0);
}
@Test
public void testAllErased() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setGroupedFields(new FieldSet(0, 2, 3));
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNull(filtered);
}
@Test
public void testGroupingPreserved1() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"0;2;3"}, null, null, tupleInfo, tupleInfo);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setGroupedFields(new FieldSet(0, 2, 3));
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNotNull(filtered);
assertNotNull(filtered.getGroupedFields());
assertEquals(3, filtered.getGroupedFields().size());
assertTrue(filtered.getGroupedFields().contains(0));
assertTrue(filtered.getGroupedFields().contains(2));
assertTrue(filtered.getGroupedFields().contains(3));
assertNull(filtered.getOrdering());
}
@Test
public void testGroupingPreserved2() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"3->0;5->2;1->3"}, null, null, tupleInfo, tupleInfo);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setGroupedFields(new FieldSet(0, 2, 3));
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNotNull(filtered);
assertNotNull(filtered.getGroupedFields());
assertEquals(3, filtered.getGroupedFields().size());
assertTrue(filtered.getGroupedFields().contains(3));
assertTrue(filtered.getGroupedFields().contains(5));
assertTrue(filtered.getGroupedFields().contains(1));
assertNull(filtered.getOrdering());
}
@Test
public void testGroupingErased() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"0;2"}, null, null, tupleInfo, tupleInfo);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setGroupedFields(new FieldSet(0, 2, 3));
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNull(filtered);
}
@Test
public void testOrderPreserved1() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"1;4;6"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(4, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setOrdering(o);
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNotNull(filtered);
assertNotNull(filtered.getOrdering());
assertEquals(3, filtered.getOrdering().getNumberOfFields());
assertEquals(4, filtered.getOrdering().getFieldNumber(0).intValue());
assertEquals(1, filtered.getOrdering().getFieldNumber(1).intValue());
assertEquals(6, filtered.getOrdering().getFieldNumber(2).intValue());
assertEquals(LongValue.class, filtered.getOrdering().getType(0));
assertEquals(IntValue.class, filtered.getOrdering().getType(1));
assertEquals(ByteValue.class, filtered.getOrdering().getType(2));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(0));
assertEquals(Order.ASCENDING, filtered.getOrdering().getOrder(1));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(2));
assertNull(filtered.getGroupedFields());
}
@Test
public void testOrderPreserved2() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"5->1;0->4;2->6"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(4, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setOrdering(o);
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNotNull(filtered);
assertNotNull(filtered.getOrdering());
assertEquals(3, filtered.getOrdering().getNumberOfFields());
assertEquals(0, filtered.getOrdering().getFieldNumber(0).intValue());
assertEquals(5, filtered.getOrdering().getFieldNumber(1).intValue());
assertEquals(2, filtered.getOrdering().getFieldNumber(2).intValue());
assertEquals(LongValue.class, filtered.getOrdering().getType(0));
assertEquals(IntValue.class, filtered.getOrdering().getType(1));
assertEquals(ByteValue.class, filtered.getOrdering().getType(2));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(0));
assertEquals(Order.ASCENDING, filtered.getOrdering().getOrder(1));
assertEquals(Order.DESCENDING, filtered.getOrdering().getOrder(2));
assertNull(filtered.getGroupedFields());
}
@Test
public void testOrderErased() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"1; 4"}, null, null, tupleInfo, tupleInfo);
Ordering o = new Ordering();
o.appendOrdering(4, LongValue.class, Order.DESCENDING);
o.appendOrdering(1, IntValue.class, Order.ASCENDING);
o.appendOrdering(6, ByteValue.class, Order.DESCENDING);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setOrdering(o);
RequestedLocalProperties filtered = rlProp.filterBySemanticProperties(sProps, 0);
assertNull(filtered);
}
@Test
public void testDualGroupingPreserved() {
DualInputSemanticProperties dprops = new DualInputSemanticProperties();
SemanticPropUtil.getSemanticPropsDualFromString(dprops, new String[]{"1->0;3;2->4"}, new String[]{"0->7;1"},
null, null, null, null, tupleInfo, tupleInfo, tupleInfo);
RequestedLocalProperties lprops1 = new RequestedLocalProperties();
lprops1.setGroupedFields(new FieldSet(0,3,4));
RequestedLocalProperties lprops2 = new RequestedLocalProperties();
lprops2.setGroupedFields(new FieldSet(7, 1));
RequestedLocalProperties filtered1 = lprops1.filterBySemanticProperties(dprops, 0);
RequestedLocalProperties filtered2 = lprops2.filterBySemanticProperties(dprops, 1);
assertNotNull(filtered1);
assertNotNull(filtered1.getGroupedFields());
assertEquals(3, filtered1.getGroupedFields().size());
assertTrue(filtered1.getGroupedFields().contains(1));
assertTrue(filtered1.getGroupedFields().contains(2));
assertTrue(filtered1.getGroupedFields().contains(3));
assertNull(filtered1.getOrdering());
assertNotNull(filtered2);
assertNotNull(filtered2.getGroupedFields());
assertEquals(2, filtered2.getGroupedFields().size());
assertTrue(filtered2.getGroupedFields().contains(0));
assertTrue(filtered2.getGroupedFields().contains(1));
assertNull(filtered2.getOrdering());
}
@Test(expected = IndexOutOfBoundsException.class)
public void testInvalidInputIndex() {
SingleInputSemanticProperties sProps = new SingleInputSemanticProperties();
SemanticPropUtil.getSemanticPropsSingleFromString(sProps, new String[]{"1; 4"}, null, null, tupleInfo, tupleInfo);
RequestedLocalProperties rlProp = new RequestedLocalProperties();
rlProp.setGroupedFields(new FieldSet(1, 4));
rlProp.filterBySemanticProperties(sProps, 1);
}
}
......@@ -278,19 +278,19 @@ public class WorksetIterationsJavaApiCompilerTest extends CompilerTestBase {
}
})
.name(JOIN_WITH_SOLUTION_SET)
.withConstantSetSecond(joinPreservesSolutionSet ? new String[] {"0->0", "1->1", "2->2" } : null);
.withForwardedFieldsSecond(joinPreservesSolutionSet ? new String[] {"0->0", "1->1", "2->2" } : null);
DataSet<Tuple3<Long, Long, Long>> nextWorkset = joinedWithSolutionSet.groupBy(1, 2)
.reduceGroup(new RichGroupReduceFunction<Tuple3<Long,Long,Long>, Tuple3<Long,Long,Long>>() {
public void reduce(Iterable<Tuple3<Long, Long, Long>> values, Collector<Tuple3<Long, Long, Long>> out) {}
})
.name(NEXT_WORKSET_REDUCER_NAME)
.withConstantSet("1->1","2->2","0->0");
.withForwardedFields("1->1","2->2","0->0");
DataSet<Tuple3<Long, Long, Long>> nextSolutionSet = mapBeforeSolutionDelta ?
joinedWithSolutionSet.map(new RichMapFunction<Tuple3<Long, Long, Long>,Tuple3<Long, Long, Long>>() { public Tuple3<Long, Long, Long> map(Tuple3<Long, Long, Long> value) { return value; } })
.name(SOLUTION_DELTA_MAPPER_NAME).withConstantSet("0->0","1->1","2->2") :
.name(SOLUTION_DELTA_MAPPER_NAME).withForwardedFields("0->0","1->1","2->2") :
joinedWithSolutionSet;
iter.closeWith(nextSolutionSet, nextWorkset)
......
......@@ -59,7 +59,7 @@ public abstract class DualInputOperator<IN1, IN2, OUT, FT extends Function> exte
/**
* Semantic properties of the associated function.
*/
private DualInputSemanticProperties semanticProperties;
private DualInputSemanticProperties semanticProperties = new DualInputSemanticProperties();
// --------------------------------------------------------------------------------------------
......
......@@ -27,20 +27,20 @@ import org.apache.flink.api.common.operators.util.FieldSet;
/**
* Container for the semantic properties associated to a dual input operator.
*/
public class DualInputSemanticProperties extends SemanticProperties {
public class DualInputSemanticProperties implements SemanticProperties {
private static final long serialVersionUID = 1L;
/**
* Mapping from fields in the source record(s) in the first input to fields
* in the destination record(s).
*/
private Map<Integer,FieldSet> forwardedFields1;
private Map<Integer,FieldSet> fieldMapping1;
/**
* Mapping from fields in the source record(s) in the second input to fields
* in the destination record(s).
*/
private Map<Integer,FieldSet> forwardedFields2;
private Map<Integer,FieldSet> fieldMapping2;
/**
* Set of fields that are read in the source record(s) from the
......@@ -56,293 +56,124 @@ public class DualInputSemanticProperties extends SemanticProperties {
public DualInputSemanticProperties() {
init();
this.fieldMapping1 = new HashMap<Integer,FieldSet>();
this.fieldMapping2 = new HashMap<Integer,FieldSet>();
this.readFields1 = null;
this.readFields2 = null;
}
/**
* Finds the source field where the given field was forwarded from.
* @param dest The destination field in the output data.
* @return FieldSet containing the source input fields.
*/
public FieldSet forwardedFrom1(int dest) {
FieldSet fs = null;
for (Map.Entry<Integer, FieldSet> entry : forwardedFields1.entrySet()) {
if (entry.getValue().contains(dest)) {
if (fs == null) {
fs = new FieldSet();
}
@Override
public FieldSet getForwardingTargetFields(int input, int sourceField) {
fs = fs.addField(entry.getKey());
}
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
} else if (input == 0) {
return fieldMapping1.containsKey(sourceField) ? fieldMapping1.get(sourceField) : FieldSet.EMPTY_SET;
} else {
return fieldMapping2.containsKey(sourceField) ? fieldMapping2.get(sourceField) : FieldSet.EMPTY_SET;
}
return fs;
}
public FieldSet forwardedFrom2(int dest) {
FieldSet fs = null;
for (Map.Entry<Integer, FieldSet> entry : forwardedFields2.entrySet()) {
if (entry.getValue().contains(dest)) {
if (fs == null) {
fs = new FieldSet();
}
@Override
public int getForwardingSourceField(int input, int targetField) {
Map<Integer, FieldSet> fieldMapping;
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
} else if (input == 0) {
fieldMapping = fieldMapping1;
} else {
fieldMapping = fieldMapping2;
}
fs = fs.addField(entry.getKey());
for (Map.Entry<Integer, FieldSet> e : fieldMapping.entrySet()) {
if (e.getValue().contains(targetField)) {
return e.getKey();
}
}
return fs;
return -1;
}
/**
* Adds, to the existing information, a field that is forwarded directly
* from the source record(s) in the first input to the destination
* record(s).
*
* @param sourceField the position in the source record(s) from the first input
* @param destinationField the position in the destination record(s)
*/
public void addForwardedField1(int sourceField, int destinationField) {
FieldSet old = this.forwardedFields1.get(sourceField);
if (old == null) {
old = FieldSet.EMPTY_SET;
@Override
public FieldSet getReadFields(int input) {
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
}
FieldSet fs = old.addField(destinationField);
this.forwardedFields1.put(sourceField, fs);
}
/**
* Adds, to the existing information, a field that is forwarded directly
* from the source record(s) in the first input to multiple fields in
* the destination record(s).
*
* @param sourceField the position in the source record(s)
* @param destinationFields the position in the destination record(s)
*/
public void addForwardedField1(int sourceField, FieldSet destinationFields) {
FieldSet old = this.forwardedFields1.get(sourceField);
if (old == null) {
old = FieldSet.EMPTY_SET;
if (input == 0) {
return readFields1;
} else {
return readFields2;
}
FieldSet fs = old.addFields(destinationFields);
this.forwardedFields1.put(sourceField, fs);
}
/**
* Sets a field that is forwarded directly from the source
* record(s) in the first input to multiple fields in the
* destination record(s).
*
* @param sourceField the position in the source record(s)
* @param destinationFields the position in the destination record(s)
*/
public void setForwardedField1(int sourceField, FieldSet destinationFields) {
this.forwardedFields1.put(sourceField, destinationFields);
}
/**
* Gets the fields in the destination record where the source
* field from the first input is forwarded.
*
* @param sourceField the position in the source record
* @return the destination fields, or null if they do not exist
*/
public FieldSet getForwardedField1(int sourceField) {
if (isAllFieldsConstant()) {
return new FieldSet(sourceField);
}
return this.forwardedFields1.get(sourceField);
}
/**
* Adds, to the existing information, a field that is forwarded directly
* from the source record(s) in the second input to the destination
* from the source record(s) in the first input to the destination
* record(s).
*
* @param sourceField the position in the source record(s) from the first input
* @param destinationField the position in the destination record(s)
*/
public void addForwardedField2(int sourceField, int destinationField) {
FieldSet old = this.forwardedFields2.get(sourceField);
if (old == null) {
old = FieldSet.EMPTY_SET;
}
FieldSet fs = old.addField(destinationField);
this.forwardedFields2.put(sourceField, fs);
}
/**
* Adds, to the existing information, a field that is forwarded directly
* from the source record(s) in the second input to multiple fields in
* the destination record(s).
*
* @param sourceField the position in the source record(s)
* @param destinationFields the position in the destination record(s)
*/
public void addForwardedField2(int sourceField, FieldSet destinationFields) {
FieldSet old = this.forwardedFields2.get(sourceField);
if (old == null) {
old = FieldSet.EMPTY_SET;
}
FieldSet fs = old.addFields(destinationFields);
this.forwardedFields2.put(sourceField, fs);
}
/**
* Sets a field that is forwarded directly from the source
* record(s) in the second input to multiple fields in the
* destination record(s).
*
* @param sourceField the position in the source record(s)
* @param destinationFields the position in the destination record(s)
*/
public void setForwardedField2(int sourceField, FieldSet destinationFields) {
this.forwardedFields2.put(sourceField, destinationFields);
}
/**
* Gets the fields in the destination record where the source
* field from the second input is forwarded.
*
*
* @param input the input of the source field
* @param sourceField the position in the source record
* @return the destination fields, or null if they do not exist
* @param targetField the position in the destination record
*/
public FieldSet getForwardedField2(int sourceField) {
if (isAllFieldsConstant()) {
return new FieldSet(sourceField);
}
public void addForwardedField(int input, int sourceField, int targetField) {
return this.forwardedFields2.get(sourceField);
}
Map<Integer, FieldSet> fieldMapping;
@Override
public FieldSet getSourceField(int input, int field) {
if (isAllFieldsConstant()) {
return new FieldSet(field);
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
} else if (input == 0) {
fieldMapping = this.fieldMapping1;
} else {
fieldMapping = this.fieldMapping2;
}
switch(input) {
case 0:
return this.forwardedFrom1(field);
case 1:
return this.forwardedFrom2(field);
default:
throw new IndexOutOfBoundsException();
if(isTargetFieldPresent(targetField, fieldMapping)) {
throw new InvalidSemanticAnnotationException("Target field "+targetField+" was added twice to input "+input);
}
}
@Override
public FieldSet getForwardFields(int input, int field) {
if (isAllFieldsConstant()) {
return new FieldSet(field);
FieldSet targetFields = fieldMapping.get(sourceField);
if (targetFields != null) {
fieldMapping.put(sourceField, targetFields.addField(targetField));
} else {
fieldMapping.put(sourceField, new FieldSet(targetField));
}
}
if (input == 0) {
return this.getForwardedField1(field);
} else if (input == 1) {
return this.getForwardedField2(field);
private boolean isTargetFieldPresent(int targetField, Map<Integer, FieldSet> fieldMapping) {
for(FieldSet targetFields : fieldMapping.values()) {
if(targetFields.contains(targetField)) {
return true;
}
}
return null;
return false;
}
/**
* Adds, to the existing information, field(s) that are read in
* the source record(s) from the first input.
*
* @param readFields the position(s) in the source record(s)
*/
public void addReadFields1(FieldSet readFields) {
if (this.readFields1 == null) {
this.readFields1 = readFields;
} else {
this.readFields1 = this.readFields2.addFields(readFields);
}
}
/**
* Sets the field(s) that are read in the source record(s) from the first
* input.
*
* @param readFields the position(s) in the source record(s)
*/
public void setReadFields1(FieldSet readFields) {
this.readFields1 = readFields;
}
/**
* Gets the field(s) in the source record(s) from the first input
* that are read.
*
* @return the field(s) in the record, or null if they are not set
*/
public FieldSet getReadFields1() {
return this.readFields1;
}
/**
* Adds, to the existing information, field(s) that are read in
* the source record(s) from the second input.
*
*
* @param input the input of the read fields
* @param readFields the position(s) in the source record(s)
*/
public void addReadFields2(FieldSet readFields) {
if (this.readFields2 == null) {
this.readFields2 = readFields;
public void addReadFields(int input, FieldSet readFields) {
FieldSet curReadFields;
if (input != 0 && input != 1) {
throw new IndexOutOfBoundsException();
} else if (input == 0) {
this.readFields1 = (this.readFields1 == null) ? readFields.clone() : this.readFields1.addFields(readFields);
} else {
this.readFields2 = this.readFields2.addFields(readFields);
this.readFields2 = (this.readFields2 == null) ? readFields.clone() : this.readFields2.addFields(readFields);
}
}
/**
* Sets the field(s) that are read in the source record(s) from the second
* input.
*
* @param readFields the position(s) in the source record(s)
*/
public void setReadFields2(FieldSet readFields) {
this.readFields2 = readFields;
}
/**
* Gets the field(s) in the source record(s) from the second input
* that are read.
*
* @return the field(s) in the record, or null if they are not set
*/
public FieldSet getReadFields2() {
return this.readFields2;
}
/**
* Clears the object.
*/
@Override
public void clearProperties() {
super.clearProperties();
init();
}
@Override
public boolean isEmpty() {
return super.isEmpty() &&
(forwardedFields1 == null || forwardedFields1.isEmpty()) &&
(forwardedFields2 == null || forwardedFields2.isEmpty()) &&
(readFields1 == null || readFields1.size() == 0) &&
(readFields2 == null || readFields2.size() == 0);
}
@Override
public String toString() {
return "DISP(" + this.forwardedFields1 + "; " + this.forwardedFields2 + ")";
return "DISP(" + this.fieldMapping1 + "; " + this.fieldMapping2 + ")";
}
private void init() {
this.forwardedFields1 = new HashMap<Integer,FieldSet>();
this.forwardedFields2 = new HashMap<Integer,FieldSet>();
this.readFields1 = null;
this.readFields2 = null;
}
}
......@@ -144,7 +144,7 @@ public class Ordering {
}
for (int i = 0; i < this.indexes.size(); i++) {
if (this.indexes.get(i).intValue() != otherOrdering.indexes.get(i).intValue()) {
if (this.indexes.get(i) != otherOrdering.indexes.get(i)) {
return false;
}
......
......@@ -21,72 +21,73 @@ package org.apache.flink.api.common.operators;
import java.io.Serializable;
import org.apache.flink.api.common.InvalidProgramException;
import org.apache.flink.api.common.operators.util.FieldSet;
/**
* Container for the semantic properties associated to an operator.
*/
public abstract class SemanticProperties implements Serializable {
private boolean allFieldsConstant;
public interface SemanticProperties extends Serializable {
private static final long serialVersionUID = 1L;
/**
* Returns the indexes of all target fields to which a source field has been
* unmodified copied by a function.
*
* @param input The input id for the requested source field (0 for first input, 1 for second input)
* @param sourceField The index of the field for which the target position index is requested.
* @return A set containing the indexes of all target fields to which the source field has been unmodified copied.
*
*/
public FieldSet getForwardingTargetFields(int input, int sourceField);
/**
* Returns the index of the source field on the given input from which the target field
* has been unmodified copied by a function.
*
* @param input The input id for the requested source field (0 for first input, 1 for second input)
* @param targetField The index of the target field to which the source field has been copied.
* @return The index of the source field on the given index that was copied to the given target field.
* -1 if the target field was not copied from any source field of the given input.
*/
public int getForwardingSourceField(int input, int targetField);
/** Set of fields that are written in the destination record(s).*/
private FieldSet writtenFields;
/**
* Adds, to the existing information, field(s) that are written in
* the destination record(s).
*
* @param writtenFields the position(s) in the destination record(s)
* Returns the position indexes of all fields of an input that are accessed by a function.
*
* @param input The input id for which accessed fields are requested.
* @return A set of fields of the specified input which have been accessed by the function. Null if no information is available.
*/
public void addWrittenFields(FieldSet writtenFields) {
if(this.writtenFields == null) {
this.writtenFields = writtenFields;
} else {
this.writtenFields = this.writtenFields.addFields(writtenFields);
public FieldSet getReadFields(int input);
// ----------------------------------------------------------------------
public static class InvalidSemanticAnnotationException extends InvalidProgramException {
public InvalidSemanticAnnotationException(String s) {
super(s);
}
}
public void setAllFieldsConstant(boolean constant) {
this.allFieldsConstant = constant;
public InvalidSemanticAnnotationException(String s, Throwable e) {
super(s,e);
}
}
public boolean isAllFieldsConstant() {
return this.allFieldsConstant;
}
public static class EmptySemanticProperties implements SemanticProperties {
@Override
public FieldSet getForwardingTargetFields(int input, int sourceField) {
return FieldSet.EMPTY_SET;
}
public abstract FieldSet getForwardFields(int input, int field);
@Override
public int getForwardingSourceField(int input, int targetField) {
return -1;
}
public abstract FieldSet getSourceField(int input, int field);
@Override
public FieldSet getReadFields(int input) {
return null;
}
/**
* Sets the field(s) that are written in the destination record(s).
*
* @param writtenFields the position(s) in the destination record(s)
*/
public void setWrittenFields(FieldSet writtenFields) {
this.writtenFields = writtenFields;
}
/**
* Gets the field(s) in the destination record(s) that are written.
*
* @return the field(s) in the record, or null if they are not set
*/
public FieldSet getWrittenFields() {
return this.writtenFields;
}
/**
* Clears the object.
*/
public void clearProperties() {
this.writtenFields = null;
}
public boolean isEmpty() {
return this.writtenFields == null || this.writtenFields.size() == 0;
}
}
......@@ -42,7 +42,7 @@ public abstract class SingleInputOperator<IN, OUT, FT extends Function> extends
private final int[] keyFields;
/** Semantic properties of the associated function. */
private SingleInputSemanticProperties semanticProperties;
private SingleInputSemanticProperties semanticProperties = new SingleInputSemanticProperties();
// --------------------------------------------------------------------------------------------
......
......@@ -26,120 +26,90 @@ import org.apache.flink.api.common.operators.util.FieldSet;
/**
* Container for the semantic properties associated to a single input operator.
*/
public class SingleInputSemanticProperties extends SemanticProperties {
public class SingleInputSemanticProperties implements SemanticProperties {
private static final long serialVersionUID = 1L;
/**Mapping from fields in the source record(s) to fields in the destination record(s). */
private Map<Integer,FieldSet> forwardedFields;
/** Set of fields that are read in the source record(s).*/
/**
* Mapping from fields in the source record(s) to fields in the destination
* record(s).
*/
private Map<Integer,FieldSet> fieldMapping;
/**
* Set of fields that are read in the source record(s).
*/
private FieldSet readFields;
public SingleInputSemanticProperties() {
this.fieldMapping = new HashMap<Integer, FieldSet>();
this.readFields = null;
}
@Override
public FieldSet getForwardFields(int input, int field) {
public FieldSet getForwardingTargetFields(int input, int sourceField) {
if (input != 0) {
throw new IndexOutOfBoundsException();
}
return this.getForwardedField(field);
return this.fieldMapping.containsKey(sourceField) ? this.fieldMapping.get(sourceField) : FieldSet.EMPTY_SET;
}
@Override
public FieldSet getSourceField(int input, int field) {
public int getForwardingSourceField(int input, int targetField) {
if (input != 0) {
throw new IndexOutOfBoundsException();
}
if (isAllFieldsConstant()) {
return new FieldSet(field);
for (Map.Entry<Integer, FieldSet> e : fieldMapping.entrySet()) {
if (e.getValue().contains(targetField)) {
return e.getKey();
}
}
return this.forwardedFrom(field);
return -1;
}
public FieldSet forwardedFrom(int dest) {
FieldSet fs = null;
for (Map.Entry<Integer, FieldSet> entry : forwardedFields.entrySet()) {
if (entry.getValue().contains(dest)) {
if (fs == null) {
fs = new FieldSet();
}
fs = fs.addField(entry.getKey());
}
@Override
public FieldSet getReadFields(int input) {
if (input != 0) {
throw new IndexOutOfBoundsException();
}
return fs;
}
public SingleInputSemanticProperties() {
init();
return this.readFields;
}
/**
* Adds, to the existing information, a field that is forwarded directly
* from the source record(s) to the destination record(s).
*
*
* @param sourceField the position in the source record(s)
* @param destinationField the position in the destination record(s)
* @param targetField the position in the destination record(s)
*/
public void addForwardedField(int sourceField, int destinationField) {
FieldSet old = this.forwardedFields.get(sourceField);
if (old == null) {
old = FieldSet.EMPTY_SET;
public void addForwardedField(int sourceField, int targetField) {
if(isTargetFieldPresent(targetField)) {
throw new InvalidSemanticAnnotationException("Target field "+targetField+" was added twice.");
}
FieldSet fs = old.addField(destinationField);
this.forwardedFields.put(sourceField, fs);
}
/**
* Adds, to the existing information, a field that is forwarded directly
* from the source record(s) to multiple fields in the destination
* record(s).
*
* @param sourceField the position in the source record(s)
* @param destinationFields the position in the destination record(s)
*/
public void addForwardedField(int sourceField, FieldSet destinationFields) {
FieldSet old = this.forwardedFields.get(sourceField);
if (old == null) {
old = FieldSet.EMPTY_SET;
FieldSet targetFields = fieldMapping.get(sourceField);
if (targetFields != null) {
fieldMapping.put(sourceField, targetFields.addField(targetField));
} else {
fieldMapping.put(sourceField, new FieldSet(targetField));
}
FieldSet fs = old.addFields(destinationFields);
this.forwardedFields.put(sourceField, fs);
}
/**
* Sets a field that is forwarded directly from the source
* record(s) to multiple fields in the destination record(s).
*
* @param sourceField the position in the source record(s)
* @param destinationFields the position in the destination record(s)
*/
public void setForwardedField(int sourceField, FieldSet destinationFields) {
this.forwardedFields.put(sourceField,destinationFields);
}
/**
* Gets the fields in the destination record where the source
* field is forwarded.
*
* @param sourceField the position in the source record
* @return the destination fields, or null if they do not exist
*/
public FieldSet getForwardedField(int sourceField) {
if (isAllFieldsConstant()) {
return new FieldSet(sourceField);
}
return this.forwardedFields.get(sourceField);
private boolean isTargetFieldPresent(int targetField) {
for(FieldSet targetFields : fieldMapping.values()) {
if(targetFields.contains(targetField)) {
return true;
}
}
return false;
}
/**
* Adds, to the existing information, field(s) that are read in
* the source record(s).
*
*
* @param readFields the position(s) in the source record(s)
*/
public void addReadFields(FieldSet readFields) {
......@@ -149,112 +119,39 @@ public class SingleInputSemanticProperties extends SemanticProperties {
this.readFields = this.readFields.addFields(readFields);
}
}
/**
* Sets the field(s) that are read in the source record(s).
*
* @param readFields the position(s) in the source record(s)
*/
public void setReadFields(FieldSet readFields) {
this.readFields = readFields;
}
/**
* Gets the field(s) in the source record(s) that are read.
*
* @return the field(s) in the record, or null if they are not set
*/
public FieldSet getReadFields() {
return this.readFields;
}
/**
* Clears the object.
*/
@Override
public void clearProperties() {
super.clearProperties();
init();
}
@Override
public boolean isEmpty() {
return super.isEmpty() &&
(forwardedFields == null || forwardedFields.isEmpty()) &&
(readFields == null || readFields.size() == 0);
}
@Override
public String toString() {
return "SISP(" + this.forwardedFields + ")";
return "SISP(" + this.fieldMapping + ")";
}
private void init() {
this.forwardedFields = new HashMap<Integer,FieldSet>();
this.readFields = null;
}
// --------------------------------------------------------------------------------------------
public static class AllFieldsConstantProperties extends SingleInputSemanticProperties {
private static final long serialVersionUID = 1L;
@Override
public FieldSet getReadFields() {
return FieldSet.EMPTY_SET;
}
@Override
public FieldSet getWrittenFields() {
return FieldSet.EMPTY_SET;
}
public static class AllFieldsForwardedProperties extends SingleInputSemanticProperties {
private static final long serialVersionUID = 1L;
@Override
public FieldSet getForwardedField(int sourceField) {
public FieldSet getForwardingTargetFields(int input, int sourceField) {
if(input != 0) {
throw new IndexOutOfBoundsException();
}
return new FieldSet(sourceField);
}
// ----- all mutating operations are unsupported -----
@Override
public void addForwardedField(int sourceField, FieldSet destinationFields) {
throw new UnsupportedOperationException();
}
@Override
public void addForwardedField(int sourceField, int destinationField) {
throw new UnsupportedOperationException();
}
@Override
public void setForwardedField(int sourceField, FieldSet destinationFields) {
throw new UnsupportedOperationException();
}
@Override
public void addReadFields(FieldSet readFields) {
throw new UnsupportedOperationException();
}
@Override
public void setReadFields(FieldSet readFields) {
throw new UnsupportedOperationException();
}
@Override
public void addWrittenFields(FieldSet writtenFields) {
throw new UnsupportedOperationException();
public int getForwardingSourceField(int input, int targetField) {
if(input != 0) {
throw new IndexOutOfBoundsException();
}
return targetField;
}
@Override
public void setWrittenFields(FieldSet writtenFields) {
throw new UnsupportedOperationException();
}
@Override
public boolean isEmpty() {
return false;
public void addForwardedField(int sourceField, int targetField) {
throw new UnsupportedOperationException("Cannot modify forwarded fields");
}
}
}
......@@ -82,7 +82,7 @@ public class PartitionOperatorBase<IN> extends SingleInputOperator<IN, IN, NoOpF
@Override
public SingleInputSemanticProperties getSemanticProperties() {
return new SingleInputSemanticProperties.AllFieldsConstantProperties();
return new SingleInputSemanticProperties.AllFieldsForwardedProperties();
}
// --------------------------------------------------------------------------------------------
......
......@@ -18,6 +18,7 @@
package org.apache.flink.api.common.typeutils;
import java.util.ArrayList;
import java.util.List;
import org.apache.flink.api.common.typeinfo.AtomicType;
......@@ -38,10 +39,41 @@ public abstract class CompositeType<T> extends TypeInformation<T> {
}
/**
* Returns the keyPosition for the given fieldPosition, offsetted by the given offset
* Returns the flat field descriptors for the given field expression.
*
* @param fieldExpression The field expression for which the flat field descriptors are computed.
* @return The list of descriptors for the flat fields which are specified by the field expression.
*/
public List<FlatFieldDescriptor> getFlatFields(String fieldExpression) {
List<FlatFieldDescriptor> result = new ArrayList<FlatFieldDescriptor>();
this.getFlatFields(fieldExpression, 0, result);
return result;
}
/**
* Computes the flat field descriptors for the given field expression with the given offset.
*
* @param fieldExpression The field expression for which the FlatFieldDescriptors are computed.
* @param offset The offset to use when computing the positions of the flat fields.
* @param result The list into which all flat field descriptors are inserted.
*/
public abstract void getFlatFields(String fieldExpression, int offset, List<FlatFieldDescriptor> result);
/**
* Returns the type of the (nested) field at the given field expression position.
* Wildcards are not allowed.
*
* @param fieldExpression The field expression for which the field of which the type is returned.
* @return The type of the field at the given field expression.
*/
public abstract <X> TypeInformation<X> getTypeAt(String fieldExpression);
/**
* Returns the type of the (unnested) field at the given field position.
*
* @param pos The position of the (unnested) field in this composite type.
* @return The type of the field at the given position.
*/
public abstract void getKey(String fieldExpression, int offset, List<FlatFieldDescriptor> result);
public abstract <X> TypeInformation<X> getTypeAt(int pos);
/**
......@@ -105,8 +137,8 @@ public abstract class CompositeType<T> extends TypeInformation<T> {
private TypeInformation<?> type;
public FlatFieldDescriptor(int keyPosition, TypeInformation<?> type) {
if( !(type instanceof AtomicType)) {
throw new IllegalArgumentException("A flattened field can only be an atomic type");
if(type instanceof CompositeType) {
throw new IllegalArgumentException("A flattened field can not be a composite type");
}
this.keyPosition = keyPosition;
this.type = type;
......@@ -126,4 +158,11 @@ public abstract class CompositeType<T> extends TypeInformation<T> {
return "FlatFieldDescriptor [position="+keyPosition+" typeInfo="+type+"]";
}
}
public static class InvalidFieldReferenceException extends IllegalArgumentException {
public InvalidFieldReferenceException(String s) {
super(s);
}
}
}
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.api.common.operators;
import org.apache.flink.api.common.operators.util.FieldSet;
import org.junit.Test;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertNotNull;
import static org.junit.Assert.assertTrue;
public class DualInputSemanticPropertiesTest {
@Test
public void testGetTargetFields() {
// first input
DualInputSemanticProperties sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 0,1);
sp.addForwardedField(0, 1,4);
sp.addForwardedField(0, 2,3);
sp.addForwardedField(0, 3,2);
assertEquals(1, sp.getForwardingTargetFields(0, 0).size());
assertEquals(1, sp.getForwardingTargetFields(0, 1).size());
assertEquals(1, sp.getForwardingTargetFields(0, 2).size());
assertEquals(1, sp.getForwardingTargetFields(0, 3).size());
assertTrue(sp.getForwardingTargetFields(0, 0).contains(1));
assertTrue(sp.getForwardingTargetFields(0, 1).contains(4));
assertTrue(sp.getForwardingTargetFields(0, 2).contains(3));
assertTrue(sp.getForwardingTargetFields(0, 3).contains(2));
assertNotNull(sp.getForwardingTargetFields(0, 4));
assertEquals(0, sp.getForwardingTargetFields(0, 4).size());
sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 0,0);
sp.addForwardedField(0, 0,4);
sp.addForwardedField(0, 1,1);
sp.addForwardedField(0, 1,2);
sp.addForwardedField(0, 1,3);
assertEquals(2, sp.getForwardingTargetFields(0, 0).size());
assertEquals(3, sp.getForwardingTargetFields(0, 1).size());
assertTrue(sp.getForwardingTargetFields(0, 0).contains(0));
assertTrue(sp.getForwardingTargetFields(0, 0).contains(4));
assertTrue(sp.getForwardingTargetFields(0, 1).contains(1));
assertTrue(sp.getForwardingTargetFields(0, 1).contains(2));
assertTrue(sp.getForwardingTargetFields(0, 1).contains(3));
assertNotNull(sp.getForwardingTargetFields(0, 2));
assertEquals(0, sp.getForwardingTargetFields(0, 2).size());
// second input
sp = new DualInputSemanticProperties();
sp.addForwardedField(1, 0,1);
sp.addForwardedField(1, 1,4);
sp.addForwardedField(1, 2,3);
sp.addForwardedField(1, 3,2);
assertEquals(1, sp.getForwardingTargetFields(1, 0).size());
assertEquals(1, sp.getForwardingTargetFields(1, 1).size());
assertEquals(1, sp.getForwardingTargetFields(1, 2).size());
assertEquals(1, sp.getForwardingTargetFields(1, 3).size());
assertTrue(sp.getForwardingTargetFields(1, 0).contains(1));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(4));
assertTrue(sp.getForwardingTargetFields(1, 2).contains(3));
assertTrue(sp.getForwardingTargetFields(1, 3).contains(2));
assertNotNull(sp.getForwardingTargetFields(1, 4));
assertEquals(0, sp.getForwardingTargetFields(1, 4).size());
sp = new DualInputSemanticProperties();
sp.addForwardedField(1, 0,0);
sp.addForwardedField(1, 0,4);
sp.addForwardedField(1, 1,1);
sp.addForwardedField(1, 1,2);
sp.addForwardedField(1, 1,3);
assertEquals(2, sp.getForwardingTargetFields(1, 0).size());
assertEquals(3, sp.getForwardingTargetFields(1, 1).size());
assertTrue(sp.getForwardingTargetFields(1, 0).contains(0));
assertTrue(sp.getForwardingTargetFields(1, 0).contains(4));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(1));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(2));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(3));
assertNotNull(sp.getForwardingTargetFields(1, 2));
assertEquals(0, sp.getForwardingTargetFields(1, 2).size());
// both inputs
sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 2,6);
sp.addForwardedField(0, 7,8);
sp.addForwardedField(1, 0,1);
sp.addForwardedField(1, 1,4);
assertEquals(1, sp.getForwardingTargetFields(0, 2).size());
assertEquals(1, sp.getForwardingTargetFields(0, 7).size());
assertEquals(1, sp.getForwardingTargetFields(1, 0).size());
assertEquals(1, sp.getForwardingTargetFields(1, 1).size());
assertTrue(sp.getForwardingTargetFields(0, 2).contains(6));
assertTrue(sp.getForwardingTargetFields(0, 7).contains(8));
assertTrue(sp.getForwardingTargetFields(1, 0).contains(1));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(4));
assertNotNull(sp.getForwardingTargetFields(0, 1));
assertNotNull(sp.getForwardingTargetFields(1, 4));
assertEquals(0, sp.getForwardingTargetFields(0, 1).size());
assertEquals(0, sp.getForwardingTargetFields(1, 4).size());
sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 0,0);
sp.addForwardedField(0, 0,4);
sp.addForwardedField(0, 3,8);
sp.addForwardedField(1, 1,1);
sp.addForwardedField(1, 1,2);
sp.addForwardedField(1, 4,8);
assertEquals(2, sp.getForwardingTargetFields(0, 0).size());
assertEquals(1, sp.getForwardingTargetFields(0, 3).size());
assertEquals(2, sp.getForwardingTargetFields(1, 1).size());
assertEquals(1, sp.getForwardingTargetFields(1, 4).size());
assertTrue(sp.getForwardingTargetFields(0, 0).contains(0));
assertTrue(sp.getForwardingTargetFields(0, 0).contains(4));
assertTrue(sp.getForwardingTargetFields(0, 3).contains(8));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(1));
assertTrue(sp.getForwardingTargetFields(1, 1).contains(1));
assertTrue(sp.getForwardingTargetFields(1, 4).contains(8));
}
@Test
public void testGetSourceField() {
// first input
DualInputSemanticProperties sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 0,1);
sp.addForwardedField(0, 1,4);
sp.addForwardedField(0, 2,3);
sp.addForwardedField(0, 3,2);
assertEquals(0, sp.getForwardingSourceField(0, 1));
assertEquals(1, sp.getForwardingSourceField(0, 4));
assertEquals(2, sp.getForwardingSourceField(0, 3));
assertEquals(3, sp.getForwardingSourceField(0, 2));
assertTrue(sp.getForwardingSourceField(0, 0) < 0);
assertTrue(sp.getForwardingSourceField(0, 5) < 0);
sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 0,0);
sp.addForwardedField(0, 0,4);
sp.addForwardedField(0, 1,1);
sp.addForwardedField(0, 1,2);
sp.addForwardedField(0, 1,3);
assertEquals(0, sp.getForwardingSourceField(0, 0));
assertEquals(0, sp.getForwardingSourceField(0, 4));
assertEquals(1, sp.getForwardingSourceField(0, 1));
assertEquals(1, sp.getForwardingSourceField(0, 2));
assertEquals(1, sp.getForwardingSourceField(0, 3));
assertTrue(sp.getForwardingSourceField(0, 5) < 0);
// second input
sp = new DualInputSemanticProperties();
sp.addForwardedField(1, 0,1);
sp.addForwardedField(1, 1,4);
sp.addForwardedField(1, 2,3);
sp.addForwardedField(1, 3,2);
assertEquals(0, sp.getForwardingSourceField(1, 1));
assertEquals(1, sp.getForwardingSourceField(1, 4));
assertEquals(2, sp.getForwardingSourceField(1, 3));
assertEquals(3, sp.getForwardingSourceField(1, 2));
assertTrue(sp.getForwardingSourceField(1, 0) < 0);
assertTrue(sp.getForwardingSourceField(1, 5) < 0);
sp = new DualInputSemanticProperties();
sp.addForwardedField(1, 0,0);
sp.addForwardedField(1, 0,4);
sp.addForwardedField(1, 1,1);
sp.addForwardedField(1, 1,2);
sp.addForwardedField(1, 1,3);
assertEquals(0, sp.getForwardingSourceField(1, 0));
assertEquals(0, sp.getForwardingSourceField(1, 4));
assertEquals(1, sp.getForwardingSourceField(1, 1));
assertEquals(1, sp.getForwardingSourceField(1, 2));
assertEquals(1, sp.getForwardingSourceField(1, 3));
assertTrue(sp.getForwardingSourceField(1, 5) < 0);
}
@Test
public void testGetReadSet() {
// first input
DualInputSemanticProperties sp = new DualInputSemanticProperties();
sp.addReadFields(0, new FieldSet(0, 1));
assertEquals(2, sp.getReadFields(0).size());
assertTrue(sp.getReadFields(0).contains(0));
assertTrue(sp.getReadFields(0).contains(1));
sp.addReadFields(0, new FieldSet(3));
assertEquals(3, sp.getReadFields(0).size());
assertTrue(sp.getReadFields(0).contains(0));
assertTrue(sp.getReadFields(0).contains(1));
assertTrue(sp.getReadFields(0).contains(3));
// second input
sp = new DualInputSemanticProperties();
sp.addReadFields(1, new FieldSet(0, 1));
assertEquals(2, sp.getReadFields(1).size());
assertTrue(sp.getReadFields(1).contains(0));
assertTrue(sp.getReadFields(1).contains(1));
sp.addReadFields(1, new FieldSet(3));
assertEquals(3, sp.getReadFields(1).size());
assertTrue(sp.getReadFields(1).contains(0));
assertTrue(sp.getReadFields(1).contains(1));
assertTrue(sp.getReadFields(1).contains(3));
}
@Test(expected = SemanticProperties.InvalidSemanticAnnotationException.class)
public void testAddForwardedFieldsTargetTwice1() {
DualInputSemanticProperties sp = new DualInputSemanticProperties();
sp.addForwardedField(0, 0, 2);
sp.addForwardedField(0, 1, 2);
}
@Test(expected = SemanticProperties.InvalidSemanticAnnotationException.class)
public void testAddForwardedFieldsTargetTwice2() {
DualInputSemanticProperties sp = new DualInputSemanticProperties();
sp.addForwardedField(1, 0, 2);
sp.addForwardedField(1, 1, 2);
}
}
......@@ -26,6 +26,7 @@ import org.apache.flink.api.common.functions.GroupReduceFunction;
import org.apache.flink.api.common.functions.JoinFunction;
import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.common.operators.Order;
import org.apache.flink.api.java.functions.FunctionAnnotation.ForwardedFields;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.util.Collector;
import org.apache.flink.api.java.DataSet;
......@@ -119,6 +120,7 @@ public class EnumTrianglesBasic {
// *************************************************************************
/** Converts a Tuple2 into an Edge */
@ForwardedFields("0;1")
public static class TupleEdgeConverter implements MapFunction<Tuple2<Integer, Integer>, Edge> {
private final Edge outEdge = new Edge();
......@@ -149,6 +151,7 @@ public class EnumTrianglesBasic {
* The first vertex of a triad is the shared vertex, the second and third vertex are ordered by vertexId.
* Assumes that input edges share the first vertex and are in ascending order of the second vertex.
*/
@ForwardedFields("0")
private static class TriadBuilder implements GroupReduceFunction<Edge, Triad> {
private final List<Integer> vertices = new ArrayList<Integer>();
private final Triad outTriad = new Triad();
......
......@@ -21,6 +21,7 @@ package org.apache.flink.examples.java.relational;
import org.apache.flink.api.common.functions.CoGroupFunction;
import org.apache.flink.api.common.functions.FilterFunction;
import org.apache.flink.api.java.functions.FunctionAnnotation.ForwardedFieldsFirst;
import org.apache.flink.api.java.tuple.Tuple1;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.api.java.tuple.Tuple3;
......@@ -234,6 +235,7 @@ public class WebLogAnalysis {
* If the first input does not provide any pairs, all pairs of the second input are emitted.
* Otherwise, no pair is emitted.
*/
@ForwardedFieldsFirst("*")
public static class AntiJoinVisits implements CoGroupFunction<Tuple3<Integer, String, Integer>, Tuple1<String>, Tuple3<Integer, String, Integer>> {
/**
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册