Skip to content

Commit

Permalink
Fix equality delete with DistinctLimit
Browse files Browse the repository at this point in the history
  • Loading branch information
wypb authored and tdcmeehan committed Mar 8, 2024
1 parent 71d827e commit 67eb17d
Show file tree
Hide file tree
Showing 2 changed files with 27 additions and 1 deletion.
Original file line number Diff line number Diff line change
Expand Up @@ -35,12 +35,14 @@
import com.facebook.presto.spi.VariableAllocator;
import com.facebook.presto.spi.function.FunctionHandle;
import com.facebook.presto.spi.function.StandardFunctionResolution;
import com.facebook.presto.spi.plan.Assignments;
import com.facebook.presto.spi.plan.ConnectorJoinNode;
import com.facebook.presto.spi.plan.EquiJoinClause;
import com.facebook.presto.spi.plan.FilterNode;
import com.facebook.presto.spi.plan.JoinType;
import com.facebook.presto.spi.plan.PlanNode;
import com.facebook.presto.spi.plan.PlanNodeIdAllocator;
import com.facebook.presto.spi.plan.ProjectNode;
import com.facebook.presto.spi.plan.TableScanNode;
import com.facebook.presto.spi.relation.CallExpression;
import com.facebook.presto.spi.relation.RowExpression;
Expand Down Expand Up @@ -251,7 +253,11 @@ public PlanNode visitTableScan(TableScanNode node, RewriteContext<Void> context)
new SpecialFormExpression(SpecialFormExpression.Form.IS_NULL, BooleanType.BOOLEAN,
new SpecialFormExpression(SpecialFormExpression.Form.COALESCE, BigintType.BIGINT, deleteVersionColumns)));

return filter;
Assignments.Builder assignmentsBuilder = Assignments.builder();
filter.getOutputVariables().stream()
.filter(variableReferenceExpression -> !variableReferenceExpression.getName().startsWith(DATA_SEQUENCE_NUMBER_COLUMN_HANDLE.getName()))
.forEach(variableReferenceExpression -> assignmentsBuilder.put(variableReferenceExpression, variableReferenceExpression));
return new ProjectNode(Optional.empty(), idAllocator.getNextId(), filter, assignmentsBuilder.build(), ProjectNode.Locality.LOCAL);
}

private static ImmutableMap<Set<Integer>, DeleteSetInfo> collectDeleteInformation(Table icebergTable,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -935,6 +935,26 @@ public void testTableWithEqualityDeleteDifferentColumnOrder(String fileFormat, b
assertQuery(session, "SELECT nationkey, comment FROM " + tableName, "SELECT nationkey, comment FROM nation WHERE name != 'ARGENTINA'");
}

@Test(dataProvider = "equalityDeleteOptions")
public void testTableWithEqualityDeleteAndGroupByAndLimit(String fileFormat, boolean joinRewriteEnabled)
throws Exception
{
Session session = deleteAsJoinEnabled(joinRewriteEnabled);
Session disable = deleteAsJoinEnabled(false);
// Specify equality delete filter with different column order from table definition
String tableName = "test_v2_equality_delete_different_order" + randomTableSuffix();
assertUpdate(session, "CREATE TABLE " + tableName + " with (format = '" + fileFormat + "') AS SELECT * FROM tpch.tiny.nation", 25);
Table icebergTable = updateTable(tableName);

writeEqualityDeleteToNationTable(icebergTable, ImmutableMap.of("regionkey", 1L, "name", "ARGENTINA"));
assertQuery(session, "SELECT * FROM " + tableName, "SELECT * FROM nation WHERE name != 'ARGENTINA'");

// Test group by
assertQuery(session, "SELECT nationkey FROM " + tableName + " group by nationkey", "VALUES(0),(2),(3),(4),(5),(6),(7),(8),(9),(10),(11),(12),(13),(14),(15),(16),(17),(18),(19),(20),(21),(22),(23),(24)");
// Test group by with limit
assertQueryWithSameQueryRunner(session, "SELECT nationkey FROM " + tableName + " group by nationkey limit 100", disable);
}

@Test(dataProvider = "equalityDeleteOptions")
public void testTableWithPositionDeleteAndEqualityDelete(String fileFormat, boolean joinRewriteEnabled)
throws Exception
Expand Down

0 comments on commit 67eb17d

Please sign in to comment.