Skip to content

Commit

Permalink
Core, Flink, Spark: Drop deprecated APIs scheduled for removal in 1.8…
Browse files Browse the repository at this point in the history
….0 (#11721)
  • Loading branch information
findepi authored Dec 10, 2024
1 parent d402f83 commit 28e8180
Show file tree
Hide file tree
Showing 17 changed files with 85 additions and 1,232 deletions.
26 changes: 26 additions & 0 deletions .palantir/revapi.yml
Original file line number Diff line number Diff line change
Expand Up @@ -1145,6 +1145,32 @@ acceptedBreaks:
new: "method org.apache.iceberg.BaseMetastoreOperations.CommitStatus org.apache.iceberg.BaseMetastoreTableOperations::checkCommitStatus(java.lang.String,\
\ org.apache.iceberg.TableMetadata)"
justification: "Removing deprecated code"
"1.7.0":
org.apache.iceberg:iceberg-core:
- code: "java.method.removed"
old: "method <T extends org.apache.iceberg.StructLike> org.apache.iceberg.deletes.PositionDeleteIndex\
\ org.apache.iceberg.deletes.Deletes::toPositionIndex(java.lang.CharSequence,\
\ java.util.List<org.apache.iceberg.io.CloseableIterable<T>>)"
justification: "Removing deprecated code"
- code: "java.method.removed"
old: "method <T extends org.apache.iceberg.StructLike> org.apache.iceberg.deletes.PositionDeleteIndex\
\ org.apache.iceberg.deletes.Deletes::toPositionIndex(java.lang.CharSequence,\
\ java.util.List<org.apache.iceberg.io.CloseableIterable<T>>, java.util.concurrent.ExecutorService)"
justification: "Removing deprecated code"
- code: "java.method.removed"
old: "method <T> org.apache.iceberg.io.CloseableIterable<T> org.apache.iceberg.deletes.Deletes::streamingFilter(org.apache.iceberg.io.CloseableIterable<T>,\
\ java.util.function.Function<T, java.lang.Long>, org.apache.iceberg.io.CloseableIterable<java.lang.Long>)"
justification: "Removing deprecated code"
- code: "java.method.removed"
old: "method <T> org.apache.iceberg.io.CloseableIterable<T> org.apache.iceberg.deletes.Deletes::streamingFilter(org.apache.iceberg.io.CloseableIterable<T>,\
\ java.util.function.Function<T, java.lang.Long>, org.apache.iceberg.io.CloseableIterable<java.lang.Long>,\
\ org.apache.iceberg.deletes.DeleteCounter)"
justification: "Removing deprecated code"
- code: "java.method.removed"
old: "method <T> org.apache.iceberg.io.CloseableIterable<T> org.apache.iceberg.deletes.Deletes::streamingMarker(org.apache.iceberg.io.CloseableIterable<T>,\
\ java.util.function.Function<T, java.lang.Long>, org.apache.iceberg.io.CloseableIterable<java.lang.Long>,\
\ java.util.function.Consumer<T>)"
justification: "Removing deprecated code"
apache-iceberg-0.14.0:
org.apache.iceberg:iceberg-api:
- code: "java.class.defaultSerializationChanged"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -214,24 +214,6 @@ protected int nextBatchOf(
}
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
public class FixedWidthTypeBinaryBatchReader extends BatchReader {
@Override
protected int nextBatchOf(
final FieldVector vector,
final int expectedBatchSize,
final int numValsInVector,
final int typeWidth,
NullabilityHolder holder) {
return vectorizedPageIterator
.fixedWidthBinaryPageReader()
.nextBatch(vector, expectedBatchSize, numValsInVector, typeWidth, holder);
}
}

public class BooleanBatchReader extends BatchReader {
@Override
protected int nextBatchOf(
Expand Down Expand Up @@ -282,14 +264,6 @@ public VarWidthTypeBatchReader varWidthTypeBatchReader() {
return new VarWidthTypeBatchReader();
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
public FixedWidthTypeBinaryBatchReader fixedWidthTypeBinaryBatchReader() {
return new FixedWidthTypeBinaryBatchReader();
}

public BooleanBatchReader booleanBatchReader() {
return new BooleanBatchReader();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,19 +137,6 @@ protected void nextVal(
}
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
class FixedWidthBinaryDictEncodedReader extends BaseDictEncodedReader {
@Override
protected void nextVal(
FieldVector vector, Dictionary dict, int idx, int currentVal, int typeWidth) {
ByteBuffer buffer = dict.decodeToBinary(currentVal).toByteBuffer();
vector.getDataBuffer().setBytes((long) idx * typeWidth, buffer);
}
}

class VarWidthBinaryDictEncodedReader extends BaseDictEncodedReader {
@Override
protected void nextVal(
Expand Down Expand Up @@ -203,14 +190,6 @@ public DoubleDictEncodedReader doubleDictEncodedReader() {
return new DoubleDictEncodedReader();
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
public FixedWidthBinaryDictEncodedReader fixedWidthBinaryDictEncodedReader() {
return new FixedWidthBinaryDictEncodedReader();
}

public VarWidthBinaryDictEncodedReader varWidthBinaryDictEncodedReader() {
return new VarWidthBinaryDictEncodedReader();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,6 @@
import java.io.IOException;
import org.apache.arrow.vector.FieldVector;
import org.apache.arrow.vector.IntVector;
import org.apache.arrow.vector.VarBinaryVector;
import org.apache.iceberg.arrow.vectorized.NullabilityHolder;
import org.apache.iceberg.parquet.BasePageIterator;
import org.apache.iceberg.parquet.ParquetUtil;
Expand Down Expand Up @@ -417,39 +416,6 @@ protected void nextDictEncodedVal(
}
}

/**
* Method for reading batches of fixed width binary type (e.g. BYTE[7]). Spark does not support
* fixed width binary data type. To work around this limitation, the data is read as fixed width
* binary from parquet and stored in a {@link VarBinaryVector} in Arrow.
*
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
class FixedWidthBinaryPageReader extends BasePageReader {
@Override
protected void nextVal(
FieldVector vector, int batchSize, int numVals, int typeWidth, NullabilityHolder holder) {
vectorizedDefinitionLevelReader
.fixedWidthBinaryReader()
.nextBatch(vector, numVals, typeWidth, batchSize, holder, plainValuesReader);
}

@Override
protected void nextDictEncodedVal(
FieldVector vector, int batchSize, int numVals, int typeWidth, NullabilityHolder holder) {
vectorizedDefinitionLevelReader
.fixedWidthBinaryReader()
.nextDictEncodedBatch(
vector,
numVals,
typeWidth,
batchSize,
holder,
dictionaryEncodedValuesReader,
dictionary);
}
}

/** Method for reading batches of booleans. */
class BooleanPageReader extends BasePageReader {
@Override
Expand Down Expand Up @@ -499,14 +465,6 @@ VarWidthTypePageReader varWidthTypePageReader() {
return new VarWidthTypePageReader();
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
FixedWidthBinaryPageReader fixedWidthBinaryPageReader() {
return new FixedWidthBinaryPageReader();
}

BooleanPageReader booleanPageReader() {
return new BooleanPageReader();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,7 +27,6 @@
import org.apache.arrow.vector.FieldVector;
import org.apache.arrow.vector.FixedSizeBinaryVector;
import org.apache.arrow.vector.IntVector;
import org.apache.arrow.vector.VarBinaryVector;
import org.apache.iceberg.arrow.vectorized.NullabilityHolder;
import org.apache.iceberg.parquet.ParquetUtil;
import org.apache.iceberg.parquet.ValuesAsBytesReader;
Expand Down Expand Up @@ -493,48 +492,6 @@ protected void nextDictEncodedVal(
}
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
class FixedWidthBinaryReader extends BaseReader {
@Override
protected void nextVal(
FieldVector vector,
int idx,
ValuesAsBytesReader valuesReader,
int typeWidth,
byte[] byteArray) {
ByteBuffer buffer = valuesReader.getBuffer(typeWidth);
((VarBinaryVector) vector)
.setSafe(
idx,
buffer.array(),
buffer.position() + buffer.arrayOffset(),
buffer.limit() - buffer.position());
}

@Override
protected void nextDictEncodedVal(
FieldVector vector,
int idx,
VectorizedDictionaryEncodedParquetValuesReader reader,
int numValuesToRead,
Dictionary dict,
NullabilityHolder nullabilityHolder,
int typeWidth,
Mode mode) {
if (Mode.RLE.equals(mode)) {
reader
.fixedWidthBinaryDictEncodedReader()
.nextBatch(vector, idx, numValuesToRead, dict, nullabilityHolder, typeWidth);
} else if (Mode.PACKED.equals(mode)) {
ByteBuffer buffer = dict.decodeToBinary(reader.readInteger()).toByteBuffer();
vector.getDataBuffer().setBytes((long) idx * typeWidth, buffer);
}
}
}

class FixedSizeBinaryReader extends BaseReader {
@Override
protected void nextVal(
Expand Down Expand Up @@ -736,14 +693,6 @@ TimestampInt96Reader timestampInt96Reader() {
return new TimestampInt96Reader();
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
FixedWidthBinaryReader fixedWidthBinaryReader() {
return new FixedWidthBinaryReader();
}

FixedSizeBinaryReader fixedSizeBinaryReader() {
return new FixedSizeBinaryReader();
}
Expand Down
78 changes: 2 additions & 76 deletions core/src/main/java/org/apache/iceberg/deletes/Deletes.java
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,6 @@
import java.io.IOException;
import java.io.UncheckedIOException;
import java.util.List;
import java.util.concurrent.ExecutorService;
import java.util.function.Consumer;
import java.util.function.Function;
import java.util.function.Predicate;
Expand All @@ -38,10 +37,8 @@
import org.apache.iceberg.types.Types;
import org.apache.iceberg.util.CharSequenceMap;
import org.apache.iceberg.util.Filter;
import org.apache.iceberg.util.ParallelIterable;
import org.apache.iceberg.util.SortedMerge;
import org.apache.iceberg.util.StructLikeSet;
import org.apache.iceberg.util.ThreadPools;

public class Deletes {

Expand Down Expand Up @@ -132,9 +129,8 @@ public static <T extends StructLike> CharSequenceMap<PositionDeleteIndex> toPosi
/**
* Builds a map of position delete indexes by path.
*
* <p>Unlike {@link #toPositionIndex(CharSequence, List)}, this method builds a position delete
* index for each referenced data file and does not filter deletes. This can be useful when the
* entire delete file content is needed (e.g. caching).
* <p>This method builds a position delete index for each referenced data file and does not filter
* deletes. This can be useful when the entire delete file content is needed (e.g. caching).
*
* @param posDeletes position deletes
* @param file the source delete file for the deletes
Expand Down Expand Up @@ -173,37 +169,6 @@ private static <T extends StructLike> CloseableIterable<Long> extractPositions(
return CloseableIterable.transform(filteredRows, row -> (Long) POSITION_ACCESSOR.get(row));
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0; use delete loaders.
*/
@Deprecated
public static <T extends StructLike> PositionDeleteIndex toPositionIndex(
CharSequence dataLocation, List<CloseableIterable<T>> deleteFiles) {
return toPositionIndex(dataLocation, deleteFiles, ThreadPools.getDeleteWorkerPool());
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0; use delete loaders.
*/
@Deprecated
public static <T extends StructLike> PositionDeleteIndex toPositionIndex(
CharSequence dataLocation,
List<CloseableIterable<T>> deleteFiles,
ExecutorService deleteWorkerPool) {
DataFileFilter<T> locationFilter = new DataFileFilter<>(dataLocation);
List<CloseableIterable<Long>> positions =
Lists.transform(
deleteFiles,
deletes ->
CloseableIterable.transform(
locationFilter.filter(deletes), row -> (Long) POSITION_ACCESSOR.get(row)));
if (positions.size() > 1 && deleteWorkerPool != null) {
return toPositionIndex(new ParallelIterable<>(positions, deleteWorkerPool));
} else {
return toPositionIndex(CloseableIterable.concat(positions));
}
}

public static PositionDeleteIndex toPositionIndex(CloseableIterable<Long> posDeletes) {
return toPositionIndex(posDeletes, ImmutableList.of());
}
Expand All @@ -219,45 +184,6 @@ private static PositionDeleteIndex toPositionIndex(
}
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
public static <T> CloseableIterable<T> streamingFilter(
CloseableIterable<T> rows,
Function<T, Long> rowToPosition,
CloseableIterable<Long> posDeletes) {
return streamingFilter(rows, rowToPosition, posDeletes, new DeleteCounter());
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
public static <T> CloseableIterable<T> streamingFilter(
CloseableIterable<T> rows,
Function<T, Long> rowToPosition,
CloseableIterable<Long> posDeletes,
DeleteCounter counter) {
PositionDeleteIndex positionIndex = toPositionIndex(posDeletes);
Predicate<T> isDeleted = row -> positionIndex.isDeleted(rowToPosition.apply(row));
return filterDeleted(rows, isDeleted, counter);
}

/**
* @deprecated since 1.7.0, will be removed in 1.8.0.
*/
@Deprecated
public static <T> CloseableIterable<T> streamingMarker(
CloseableIterable<T> rows,
Function<T, Long> rowToPosition,
CloseableIterable<Long> posDeletes,
Consumer<T> markRowDeleted) {
PositionDeleteIndex positionIndex = toPositionIndex(posDeletes);
Predicate<T> isDeleted = row -> positionIndex.isDeleted(rowToPosition.apply(row));
return markDeleted(rows, isDeleted, markRowDeleted);
}

public static CloseableIterable<Long> deletePositions(
CharSequence dataLocation, CloseableIterable<StructLike> deleteFile) {
return deletePositions(dataLocation, ImmutableList.of(deleteFile));
Expand Down
Loading

0 comments on commit 28e8180

Please sign in to comment.