Skip to content
Permalink

Comparing changes

This is a direct comparison between two commits made in this repository or its related repositories. View the default comparison for this range or learn more about diff comparisons.

Open a pull request

Create a new pull request by comparing changes across two branches. If you need to, you can also . Learn more about diff comparisons here.
base repository: apache/iceberg
Failed to load repositories. Confirm that selected base ref is valid, then try again.
Loading
base: c70bfdbef5a6c2f2dadf6b92267d9ee577955f3c
Choose a base ref
..
head repository: apache/iceberg
Failed to load repositories. Confirm that selected head ref is valid, then try again.
Loading
compare: 0fb145d5dbdfa075ae87bc8c7c65d8a770a96fff
Choose a head ref
Showing with 5,628 additions and 1,366 deletions.
  1. +1 −1 .github/workflows/open-api.yml
  2. +1 −1 api/src/main/java/org/apache/iceberg/PartitionField.java
  3. +32 −7 api/src/main/java/org/apache/iceberg/PartitionSpec.java
  4. +1 −1 api/src/main/java/org/apache/iceberg/SortField.java
  5. +10 −2 api/src/main/java/org/apache/iceberg/UnboundPartitionSpec.java
  6. +10 −17 api/src/main/java/org/apache/iceberg/transforms/Dates.java
  7. +3 −1 api/src/main/java/org/apache/iceberg/transforms/ProjectionUtil.java
  8. +12 −20 api/src/main/java/org/apache/iceberg/transforms/Timestamps.java
  9. +2 −2 api/src/main/java/org/apache/iceberg/transforms/Transforms.java
  10. +1 −1 api/src/main/java/org/apache/iceberg/types/Comparators.java
  11. +1 −1 api/src/main/java/org/apache/iceberg/types/Types.java
  12. +55 −0 {core → api}/src/main/java/org/apache/iceberg/util/DateTimeUtil.java
  13. +3 −3 api/src/test/java/org/apache/iceberg/expressions/TestExpressionUtil.java
  14. +2 −2 api/src/test/java/org/apache/iceberg/expressions/TestInclusiveMetricsEvaluator.java
  15. +5 −5 api/src/test/java/org/apache/iceberg/expressions/TestMetricsEvaluatorsNaNHandling.java
  16. +2 −2 api/src/test/java/org/apache/iceberg/expressions/TestStrictMetricsEvaluator.java
  17. 0 {core → api}/src/test/java/org/apache/iceberg/util/TestDateTimeUtil.java
  18. +2 −2 aws/src/main/java/org/apache/iceberg/aws/glue/GlueCatalog.java
  19. +6 −6 {open-api → aws/src/main/resources}/s3-signer-open-api.yaml
  20. +1 −1 core/src/main/java/org/apache/iceberg/ManifestFiles.java
  21. +1 −1 core/src/main/java/org/apache/iceberg/MetadataUpdate.java
  22. +1 −1 core/src/main/java/org/apache/iceberg/MetadataUpdateParser.java
  23. +1 −1 core/src/main/java/org/apache/iceberg/PartitionData.java
  24. +95 −29 core/src/main/java/org/apache/iceberg/Partitioning.java
  25. +11 −0 core/src/main/java/org/apache/iceberg/TableMetadata.java
  26. +3 −3 core/src/main/java/org/apache/iceberg/avro/AvroFileAppender.java
  27. +2 −2 core/src/main/java/org/apache/iceberg/jdbc/JdbcCatalog.java
  28. +1 −0 core/src/main/java/org/apache/iceberg/rest/RESTSessionCatalog.java
  29. +1 −1 core/src/main/java/org/apache/iceberg/rest/requests/UpdateRequirementParser.java
  30. +1 −1 core/src/main/java/org/apache/iceberg/schema/UnionByNameVisitor.java
  31. +8 −10 core/src/main/java/org/apache/iceberg/util/SortOrderUtil.java
  32. +1 −1 core/src/test/java/org/apache/iceberg/TestManifestCaching.java
  33. +211 −10 core/src/test/java/org/apache/iceberg/TestPartitioning.java
  34. +1 −1 core/src/test/java/org/apache/iceberg/hadoop/HadoopTableTestBase.java
  35. +1 −1 core/src/test/java/org/apache/iceberg/rest/RESTCatalogAdapter.java
  36. +1 −1 dev/source-release.sh
  37. +2 −1 docs/flink-getting-started.md
  38. +8 −3 flink/v1.14/flink/src/main/java/org/apache/iceberg/flink/FlinkCatalog.java
  39. +22 −3 flink/v1.14/flink/src/main/java/org/apache/iceberg/flink/FlinkCatalogFactory.java
  40. +1 −1 flink/v1.14/flink/src/main/java/org/apache/iceberg/flink/FlinkFilters.java
  41. +33 −0 flink/v1.14/flink/src/main/java/org/apache/iceberg/flink/util/FlinkPackage.java
  42. +2 −3 flink/v1.14/flink/src/test/java/org/apache/iceberg/flink/TestFlinkCatalogTablePartitions.java
  43. +1 −1 flink/v1.14/flink/src/test/java/org/apache/iceberg/flink/source/TestFlinkScan.java
  44. +31 −0 flink/v1.14/flink/src/test/java/org/apache/iceberg/flink/util/TestFlinkPackage.java
  45. +8 −3 flink/v1.15/flink/src/main/java/org/apache/iceberg/flink/FlinkCatalog.java
  46. +22 −3 flink/v1.15/flink/src/main/java/org/apache/iceberg/flink/FlinkCatalogFactory.java
  47. +1 −1 flink/v1.15/flink/src/main/java/org/apache/iceberg/flink/FlinkFilters.java
  48. +33 −0 flink/v1.15/flink/src/main/java/org/apache/iceberg/flink/util/FlinkPackage.java
  49. +2 −3 flink/v1.15/flink/src/test/java/org/apache/iceberg/flink/TestFlinkCatalogTablePartitions.java
  50. +1 −1 flink/v1.15/flink/src/test/java/org/apache/iceberg/flink/source/TestFlinkScan.java
  51. +31 −0 flink/v1.15/flink/src/test/java/org/apache/iceberg/flink/util/TestFlinkPackage.java
  52. +8 −3 flink/v1.16/flink/src/main/java/org/apache/iceberg/flink/FlinkCatalog.java
  53. +22 −3 flink/v1.16/flink/src/main/java/org/apache/iceberg/flink/FlinkCatalogFactory.java
  54. +1 −1 flink/v1.16/flink/src/main/java/org/apache/iceberg/flink/FlinkFilters.java
  55. +10 −1 flink/v1.16/flink/src/main/java/org/apache/iceberg/flink/data/FlinkParquetReaders.java
  56. +33 −0 flink/v1.16/flink/src/main/java/org/apache/iceberg/flink/util/FlinkPackage.java
  57. +2 −3 flink/v1.16/flink/src/test/java/org/apache/iceberg/flink/TestFlinkCatalogTablePartitions.java
  58. +51 −0 flink/v1.16/flink/src/test/java/org/apache/iceberg/flink/source/TestFlinkInputFormat.java
  59. +1 −1 flink/v1.16/flink/src/test/java/org/apache/iceberg/flink/source/TestFlinkScan.java
  60. +31 −0 flink/v1.16/flink/src/test/java/org/apache/iceberg/flink/util/TestFlinkPackage.java
  61. +0 −9 hive-metastore/src/main/java/org/apache/iceberg/hive/HiveCatalog.java
  62. +4 −13 hive-metastore/src/test/java/org/apache/iceberg/hive/TestHiveCatalog.java
  63. +1 −1 nessie/src/test/java/org/apache/iceberg/nessie/BaseTestIceberg.java
  64. +10 −1 parquet/src/main/java/org/apache/iceberg/data/parquet/BaseParquetReaders.java
  65. +43 −2 parquet/src/main/java/org/apache/iceberg/parquet/ParquetValueReaders.java
  66. +1 −1 parquet/src/main/java/org/apache/iceberg/parquet/ParquetWriteSupport.java
  67. +10 −1 pig/src/main/java/org/apache/iceberg/pig/PigParquetReader.java
  68. +1 −1 python/.pre-commit-config.yaml
  69. +1 −1 python/Makefile
  70. +10 −1 python/mkdocs/docs/index.md
  71. +481 −147 python/poetry.lock
  72. +5 −4 python/pyiceberg/avro/reader.py
  73. +67 −5 python/pyiceberg/catalog/__init__.py
  74. +549 −0 python/pyiceberg/catalog/glue.py
  75. +1 −1 python/pyiceberg/catalog/hive.py
  76. +4 −4 python/pyiceberg/catalog/rest.py
  77. +2 −2 python/pyiceberg/cli/output.py
  78. +4 −0 python/pyiceberg/exceptions.py
  79. +424 −199 python/pyiceberg/expressions/__init__.py
  80. +83 −91 python/pyiceberg/expressions/literals.py
  81. +139 −65 python/pyiceberg/expressions/visitors.py
  82. +0 −15 python/pyiceberg/files.py
  83. +2 −2 python/pyiceberg/io/fsspec.py
  84. +1 −1 python/pyiceberg/io/pyarrow.py
  85. +10 −2 python/pyiceberg/manifest.py
  86. +2 −2 python/pyiceberg/{table → }/partitioning.py
  87. +2 −2 python/pyiceberg/schema.py
  88. +166 −14 python/pyiceberg/table/__init__.py
  89. +2 −2 python/pyiceberg/table/metadata.py
  90. +1 −1 python/pyiceberg/table/snapshots.py
  91. +2 −2 python/pyiceberg/table/sorting.py
  92. +207 −25 python/pyiceberg/transforms.py
  93. +44 −2 python/pyiceberg/typedef.py
  94. +7 −5 python/pyiceberg/utils/bin_packing.py
  95. +2 −6 python/pyiceberg/utils/config.py
  96. +6 −0 python/pyiceberg/utils/datetime.py
  97. +1 −1 python/pyiceberg/utils/deprecated.py
  98. +1 −1 python/pyiceberg/utils/parsing.py
  99. +4 −2 python/pyiceberg/utils/schema_conversion.py
  100. +1 −1 python/pyiceberg/utils/singleton.py
  101. +29 −0 python/pyproject.toml
  102. +3 −2 python/tests/avro/test_decoder.py
  103. +324 −0 python/tests/catalog/integration_test_glue.py
  104. +1 −1 python/tests/catalog/test_base.py
  105. +471 −0 python/tests/catalog/test_glue.py
  106. +1 −1 python/tests/catalog/test_hive.py
  107. +1 −1 python/tests/catalog/test_rest.py
  108. +1 −1 python/tests/cli/test_console.py
  109. +113 −0 python/tests/conftest.py
  110. +179 −0 python/tests/expressions/test_evaluator.py
  111. +349 −168 python/tests/expressions/test_expressions.py
  112. +110 −15 python/tests/expressions/test_literals.py
  113. +265 −338 python/tests/expressions/test_visitors.py
  114. +8 −8 python/tests/table/test_init.py
  115. +1 −1 python/tests/table/test_metadata.py
  116. +1 −1 python/tests/table/test_partitioning.py
  117. +1 −1 python/tests/table/test_snapshots.py
  118. +1 −2 python/tests/test_schema.py
  119. +390 −2 python/tests/test_transforms.py
  120. +1 −1 python/tests/utils/test_manifest.py
  121. +1 −1 spark/v2.4/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithReordering.java
  122. +1 −1 spark/v2.4/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithoutReordering.java
  123. +1 −1 spark/v2.4/spark/src/main/java/org/apache/iceberg/spark/SparkFilters.java
  124. +1 −0 spark/v3.1/spark/src/jmh/java/org/apache/iceberg/spark/source/WritersBenchmark.java
  125. +1 −1 spark/v3.1/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithReordering.java
  126. +1 −1 spark/v3.1/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithoutReordering.java
  127. +1 −1 spark/v3.1/spark/src/main/java/org/apache/iceberg/spark/SparkFilters.java
  128. +1 −1 spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithReordering.java
  129. +1 −1 spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithoutReordering.java
  130. +1 −1 spark/v3.2/spark/src/main/java/org/apache/iceberg/spark/SparkFilters.java
  131. +1 −1 spark/v3.3/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithReordering.java
  132. +1 −1 spark/v3.3/spark/src/main/java/org/apache/iceberg/spark/PruneColumnsWithoutReordering.java
  133. +1 −1 spark/v3.3/spark/src/main/java/org/apache/iceberg/spark/SparkAggregates.java
  134. +1 −1 spark/v3.3/spark/src/main/java/org/apache/iceberg/spark/SparkFilters.java
  135. +1 −1 spark/v3.3/spark/src/main/java/org/apache/iceberg/spark/SparkV2Filters.java
  136. +10 −1 spark/v3.3/spark/src/main/java/org/apache/iceberg/spark/data/SparkParquetReaders.java
  137. +74 −0 spark/v3.3/spark/src/test/java/org/apache/iceberg/spark/source/ComplexRecord.java
  138. +77 −0 spark/v3.3/spark/src/test/java/org/apache/iceberg/spark/source/NestedRecord.java
  139. +54 −0 spark/v3.3/spark/src/test/java/org/apache/iceberg/spark/source/TestPartitionValues.java
2 changes: 1 addition & 1 deletion .github/workflows/open-api.yml
Original file line number Diff line number Diff line change
@@ -50,5 +50,5 @@ jobs:
working-directory: ./open-api
run: openapi-spec-validator rest-catalog-open-api.yaml
- name: Validate S3 REST Signer spec
working-directory: ./open-api
working-directory: ./aws/src/main/resources
run: openapi-spec-validator s3-signer-open-api.yaml
2 changes: 1 addition & 1 deletion api/src/main/java/org/apache/iceberg/PartitionField.java
Original file line number Diff line number Diff line change
@@ -73,7 +73,7 @@ public boolean equals(Object other) {
return sourceId == that.sourceId
&& fieldId == that.fieldId
&& name.equals(that.name)
&& transform.equals(that.transform);
&& transform.toString().equals(that.transform.toString());
}

@Override
39 changes: 32 additions & 7 deletions api/src/main/java/org/apache/iceberg/PartitionSpec.java
Original file line number Diff line number Diff line change
@@ -419,7 +419,10 @@ Builder identity(String sourceName, String targetName) {
checkAndAddPartitionName(targetName, sourceColumn.fieldId());
PartitionField field =
new PartitionField(
sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.identity());
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.identity(sourceColumn.type()));
checkForRedundantPartitions(field);
fields.add(field);
return this;
@@ -433,7 +436,11 @@ public Builder year(String sourceName, String targetName) {
checkAndAddPartitionName(targetName);
Types.NestedField sourceColumn = findSourceColumn(sourceName);
PartitionField field =
new PartitionField(sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.year());
new PartitionField(
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.year(sourceColumn.type()));
checkForRedundantPartitions(field);
fields.add(field);
return this;
@@ -447,7 +454,11 @@ public Builder month(String sourceName, String targetName) {
checkAndAddPartitionName(targetName);
Types.NestedField sourceColumn = findSourceColumn(sourceName);
PartitionField field =
new PartitionField(sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.month());
new PartitionField(
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.month(sourceColumn.type()));
checkForRedundantPartitions(field);
fields.add(field);
return this;
@@ -461,7 +472,11 @@ public Builder day(String sourceName, String targetName) {
checkAndAddPartitionName(targetName);
Types.NestedField sourceColumn = findSourceColumn(sourceName);
PartitionField field =
new PartitionField(sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.day());
new PartitionField(
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.day(sourceColumn.type()));
checkForRedundantPartitions(field);
fields.add(field);
return this;
@@ -475,7 +490,11 @@ public Builder hour(String sourceName, String targetName) {
checkAndAddPartitionName(targetName);
Types.NestedField sourceColumn = findSourceColumn(sourceName);
PartitionField field =
new PartitionField(sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.hour());
new PartitionField(
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.hour(sourceColumn.type()));
checkForRedundantPartitions(field);
fields.add(field);
return this;
@@ -490,7 +509,10 @@ public Builder bucket(String sourceName, int numBuckets, String targetName) {
Types.NestedField sourceColumn = findSourceColumn(sourceName);
fields.add(
new PartitionField(
sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.bucket(numBuckets)));
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.bucket(sourceColumn.type(), numBuckets)));
return this;
}

@@ -503,7 +525,10 @@ public Builder truncate(String sourceName, int width, String targetName) {
Types.NestedField sourceColumn = findSourceColumn(sourceName);
fields.add(
new PartitionField(
sourceColumn.fieldId(), nextFieldId(), targetName, Transforms.truncate(width)));
sourceColumn.fieldId(),
nextFieldId(),
targetName,
Transforms.truncate(sourceColumn.type(), width)));
return this;
}

2 changes: 1 addition & 1 deletion api/src/main/java/org/apache/iceberg/SortField.java
Original file line number Diff line number Diff line change
@@ -96,7 +96,7 @@ public boolean equals(Object other) {
}

SortField that = (SortField) other;
return transform.equals(that.transform)
return transform.toString().equals(that.transform.toString())
&& sourceId == that.sourceId
&& direction == that.direction
&& nullOrder == that.nullOrder;
12 changes: 10 additions & 2 deletions api/src/main/java/org/apache/iceberg/UnboundPartitionSpec.java
Original file line number Diff line number Diff line change
@@ -22,6 +22,7 @@
import org.apache.iceberg.relocated.com.google.common.collect.Lists;
import org.apache.iceberg.transforms.Transform;
import org.apache.iceberg.transforms.Transforms;
import org.apache.iceberg.types.Type;

public class UnboundPartitionSpec {

@@ -53,10 +54,17 @@ private PartitionSpec.Builder copyToBuilder(Schema schema) {
PartitionSpec.Builder builder = PartitionSpec.builderFor(schema).withSpecId(specId);

for (UnboundPartitionField field : fields) {
Type fieldType = schema.findType(field.sourceId);
Transform<?, ?> transform;
if (fieldType != null) {
transform = Transforms.fromString(fieldType, field.transform.toString());
} else {
transform = Transforms.fromString(field.transform.toString());
}
if (field.partitionId != null) {
builder.add(field.sourceId, field.partitionId, field.name, field.transform);
builder.add(field.sourceId, field.partitionId, field.name, transform);
} else {
builder.add(field.sourceId, field.name, field.transform);
builder.add(field.sourceId, field.name, transform);
}
}

27 changes: 10 additions & 17 deletions api/src/main/java/org/apache/iceberg/transforms/Dates.java
Original file line number Diff line number Diff line change
@@ -18,9 +18,6 @@
*/
package org.apache.iceberg.transforms;

import java.time.Instant;
import java.time.LocalDate;
import java.time.ZoneOffset;
import java.time.temporal.ChronoUnit;
import org.apache.iceberg.expressions.BoundPredicate;
import org.apache.iceberg.expressions.BoundTransform;
@@ -30,6 +27,7 @@
import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.types.Type;
import org.apache.iceberg.types.Types;
import org.apache.iceberg.util.DateTimeUtil;
import org.apache.iceberg.util.SerializableFunction;

enum Dates implements Transform<Integer, Integer> {
@@ -50,24 +48,19 @@ public Integer apply(Integer days) {
return null;
}

if (granularity == ChronoUnit.DAYS) {
return days;
}

if (days >= 0) {
LocalDate date = EPOCH.plusDays(days);
return (int) granularity.between(EPOCH, date);
} else {
// add 1 day to the value to account for the case where there is exactly 1 unit between the
// date and epoch because the result will always be decremented.
LocalDate date = EPOCH.plusDays(days + 1);
return (int) granularity.between(EPOCH, date) - 1;
switch (granularity) {
case YEARS:
return DateTimeUtil.daysToYears(days);
case MONTHS:
return DateTimeUtil.daysToMonths(days);
case DAYS:
return days;
default:
throw new UnsupportedOperationException("Unsupported time unit: " + granularity);
}
}
}

private static final LocalDate EPOCH =
Instant.ofEpochSecond(0).atOffset(ZoneOffset.UTC).toLocalDate();
private final ChronoUnit granularity;
private final String name;
private final Apply apply;
Original file line number Diff line number Diff line change
@@ -231,7 +231,9 @@ static <S, T> UnboundPredicate<T> truncateArrayStrict(
static <T> UnboundPredicate<T> projectTransformPredicate(
Transform<?, T> transform, String partitionName, BoundPredicate<?> pred) {
if (pred.term() instanceof BoundTransform
&& transform.equals(((BoundTransform<?, ?>) pred.term()).transform())) {
&& transform
.toString()
.equals(((BoundTransform<?, ?>) pred.term()).transform().toString())) {
// the bound value must be a T because the transform matches
return (UnboundPredicate<T>) removeTransform(partitionName, pred);
}
32 changes: 12 additions & 20 deletions api/src/main/java/org/apache/iceberg/transforms/Timestamps.java
Original file line number Diff line number Diff line change
@@ -18,9 +18,6 @@
*/
package org.apache.iceberg.transforms;

import java.time.Instant;
import java.time.OffsetDateTime;
import java.time.ZoneOffset;
import java.time.temporal.ChronoUnit;
import org.apache.iceberg.expressions.BoundPredicate;
import org.apache.iceberg.expressions.BoundTransform;
@@ -30,6 +27,7 @@
import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
import org.apache.iceberg.types.Type;
import org.apache.iceberg.types.Types;
import org.apache.iceberg.util.DateTimeUtil;
import org.apache.iceberg.util.SerializableFunction;

enum Timestamps implements Transform<Long, Integer> {
@@ -51,27 +49,21 @@ public Integer apply(Long timestampMicros) {
return null;
}

if (timestampMicros >= 0) {
OffsetDateTime timestamp =
Instant.ofEpochSecond(
Math.floorDiv(timestampMicros, 1_000_000),
Math.floorMod(timestampMicros, 1_000_000) * 1000)
.atOffset(ZoneOffset.UTC);
return (int) granularity.between(EPOCH, timestamp);
} else {
// add 1 micro to the value to account for the case where there is exactly 1 unit between
// the timestamp and epoch because the result will always be decremented.
OffsetDateTime timestamp =
Instant.ofEpochSecond(
Math.floorDiv(timestampMicros, 1_000_000),
Math.floorMod(timestampMicros + 1, 1_000_000) * 1000)
.atOffset(ZoneOffset.UTC);
return (int) granularity.between(EPOCH, timestamp) - 1;
switch (granularity) {
case YEARS:
return DateTimeUtil.microsToYears(timestampMicros);
case MONTHS:
return DateTimeUtil.microsToMonths(timestampMicros);
case DAYS:
return DateTimeUtil.microsToDays(timestampMicros);
case HOURS:
return DateTimeUtil.microsToHours(timestampMicros);
default:
throw new UnsupportedOperationException("Unsupported time unit: " + granularity);
}
}
}

private static final OffsetDateTime EPOCH = Instant.ofEpochSecond(0).atOffset(ZoneOffset.UTC);
private final ChronoUnit granularity;
private final String name;
private final SerializableFunction<Long, Integer> apply;
Original file line number Diff line number Diff line change
@@ -46,7 +46,7 @@ private Transforms() {}
int parsedWidth = Integer.parseInt(widthMatcher.group(2));
if (name.equalsIgnoreCase("truncate")) {
return Truncate.get(parsedWidth);
} else if (name.equals("bucket")) {
} else if (name.equalsIgnoreCase("bucket")) {
return Bucket.get(parsedWidth);
}
}
@@ -75,7 +75,7 @@ private Transforms() {}
int parsedWidth = Integer.parseInt(widthMatcher.group(2));
if (name.equalsIgnoreCase("truncate")) {
return (Transform<?, ?>) Truncate.get(type, parsedWidth);
} else if (name.equals("bucket")) {
} else if (name.equalsIgnoreCase("bucket")) {
return (Transform<?, ?>) Bucket.get(type, parsedWidth);
}
}
Original file line number Diff line number Diff line change
@@ -44,7 +44,7 @@ private Comparators() {}
.put(Types.StringType.get(), Comparators.charSequences())
.put(Types.UUIDType.get(), Comparator.naturalOrder())
.put(Types.BinaryType.get(), Comparators.unsignedBytes())
.build();
.buildOrThrow();

public static Comparator<StructLike> forType(Types.StructType struct) {
return new StructLikeComparator(struct);
2 changes: 1 addition & 1 deletion api/src/main/java/org/apache/iceberg/types/Types.java
Original file line number Diff line number Diff line change
@@ -51,7 +51,7 @@ private Types() {}
.put(StringType.get().toString(), StringType.get())
.put(UUIDType.get().toString(), UUIDType.get())
.put(BinaryType.get().toString(), BinaryType.get())
.build();
.buildOrThrow();

private static final Pattern FIXED = Pattern.compile("fixed\\[(\\d+)\\]");
private static final Pattern DECIMAL = Pattern.compile("decimal\\((\\d+),\\s+(\\d+)\\)");
Original file line number Diff line number Diff line change
@@ -34,6 +34,7 @@ private DateTimeUtil() {}
public static final OffsetDateTime EPOCH = Instant.ofEpochSecond(0).atOffset(ZoneOffset.UTC);
public static final LocalDate EPOCH_DAY = EPOCH.toLocalDate();
public static final long MICROS_PER_MILLIS = 1000L;
public static final long MICROS_PER_SECOND = 1_000_000L;

public static LocalDate dateFromDays(int daysFromEpoch) {
return ChronoUnit.DAYS.addTo(EPOCH_DAY, daysFromEpoch);
@@ -133,4 +134,58 @@ public static long isoTimestampToMicros(String timestampString) {
return microsFromTimestamp(
LocalDateTime.parse(timestampString, DateTimeFormatter.ISO_LOCAL_DATE_TIME));
}

public static int daysToYears(int days) {
return convertDays(days, ChronoUnit.YEARS);
}

public static int daysToMonths(int days) {
return convertDays(days, ChronoUnit.MONTHS);
}

private static int convertDays(int days, ChronoUnit granularity) {
if (days >= 0) {
LocalDate date = EPOCH_DAY.plusDays(days);
return (int) granularity.between(EPOCH_DAY, date);
} else {
// add 1 day to the value to account for the case where there is exactly 1 unit between the
// date and epoch because the result will always be decremented.
LocalDate date = EPOCH_DAY.plusDays(days + 1);
return (int) granularity.between(EPOCH_DAY, date) - 1;
}
}

public static int microsToYears(long micros) {
return convertMicros(micros, ChronoUnit.YEARS);
}

public static int microsToMonths(long micros) {
return convertMicros(micros, ChronoUnit.MONTHS);
}

public static int microsToDays(long micros) {
return convertMicros(micros, ChronoUnit.DAYS);
}

public static int microsToHours(long micros) {
return convertMicros(micros, ChronoUnit.HOURS);
}

private static int convertMicros(long micros, ChronoUnit granularity) {
if (micros >= 0) {
long epochSecond = Math.floorDiv(micros, MICROS_PER_SECOND);
long nanoAdjustment = Math.floorMod(micros, MICROS_PER_SECOND) * 1000;
return (int) granularity.between(EPOCH, toOffsetDateTime(epochSecond, nanoAdjustment));
} else {
// add 1 micro to the value to account for the case where there is exactly 1 unit between
// the timestamp and epoch because the result will always be decremented.
long epochSecond = Math.floorDiv(micros, MICROS_PER_SECOND);
long nanoAdjustment = Math.floorMod(micros + 1, MICROS_PER_SECOND) * 1000;
return (int) granularity.between(EPOCH, toOffsetDateTime(epochSecond, nanoAdjustment)) - 1;
}
}

private static OffsetDateTime toOffsetDateTime(long epochSecond, long nanoAdjustment) {
return Instant.ofEpochSecond(epochSecond, nanoAdjustment).atOffset(ZoneOffset.UTC);
}
}
Original file line number Diff line number Diff line change
@@ -538,7 +538,7 @@ public void testSanitizeTimestamptzFuture() {

@Test
public void testSanitizeDateToday() {
String today = LocalDate.now().toString();
String today = LocalDate.now(ZoneOffset.UTC).toString();

assertEquals(
Expressions.equal("test", "(date-today)"),
@@ -558,7 +558,7 @@ public void testSanitizeDateToday() {

@Test
public void testSanitizeDateLastWeek() {
String lastWeek = LocalDate.now().minusWeeks(1).toString();
String lastWeek = LocalDate.now(ZoneOffset.UTC).minusWeeks(1).toString();

assertEquals(
Expressions.equal("test", "(date-7-days-ago)"),
@@ -578,7 +578,7 @@ public void testSanitizeDateLastWeek() {

@Test
public void testSanitizeDateNextWeek() {
String nextWeek = LocalDate.now().plusWeeks(1).toString();
String nextWeek = LocalDate.now(ZoneOffset.UTC).plusWeeks(1).toString();

assertEquals(
Expressions.equal("test", "(date-7-days-from-now)"),
Loading