Skip to content

Commit

Permalink
[fix] Fix comment
Browse files Browse the repository at this point in the history
  • Loading branch information
tsreaper committed Mar 15, 2024
1 parent d7908cc commit c051721
Show file tree
Hide file tree
Showing 35 changed files with 87 additions and 94 deletions.
8 changes: 3 additions & 5 deletions paimon-core/src/main/java/org/apache/paimon/Snapshot.java
Original file line number Diff line number Diff line change
Expand Up @@ -407,22 +407,20 @@ public Long totalRecordCount(FileStoreScan scan) {
}

public static long recordCount(List<ManifestEntry> manifestEntries) {
return manifestEntries.stream()
.mapToLong(manifest -> manifest.file().totalRowCount())
.sum();
return manifestEntries.stream().mapToLong(manifest -> manifest.file().rowCount()).sum();
}

public static long recordCountAdd(List<ManifestEntry> manifestEntries) {
return manifestEntries.stream()
.filter(manifestEntry -> FileKind.ADD.equals(manifestEntry.kind()))
.mapToLong(manifest -> manifest.file().totalRowCount())
.mapToLong(manifest -> manifest.file().rowCount())
.sum();
}

public static long recordCountDelete(List<ManifestEntry> manifestEntries) {
return manifestEntries.stream()
.filter(manifestEntry -> FileKind.DELETE.equals(manifestEntry.kind()))
.mapToLong(manifest -> manifest.file().totalRowCount())
.mapToLong(manifest -> manifest.file().rowCount())
.sum();
}

Expand Down
56 changes: 28 additions & 28 deletions paimon-core/src/main/java/org/apache/paimon/io/DataFileMeta.java
Original file line number Diff line number Diff line change
Expand Up @@ -65,8 +65,8 @@ public class DataFileMeta {
// Why don't we keep addRowCount and deleteRowCount?
// Because in previous versions of DataFileMeta, we only keep rowCount.
// We have to keep the compatibility.
private final long totalRowCount;
private final @Nullable Long addRowCount;
private final long rowCount;
private final @Nullable Long deleteRowCount;

private final BinaryRow minKey;
private final BinaryRow maxKey;
Expand All @@ -93,7 +93,7 @@ public static DataFileMeta forAppend(
fileName,
fileSize,
rowCount,
rowCount,
0L,
EMPTY_MIN_KEY,
EMPTY_MAX_KEY,
EMPTY_KEY_STATS,
Expand All @@ -107,8 +107,8 @@ public static DataFileMeta forAppend(
public DataFileMeta(
String fileName,
long fileSize,
long totalRowCount,
@Nullable Long addRowCount,
long rowCount,
@Nullable Long deleteRowCount,
BinaryRow minKey,
BinaryRow maxKey,
BinaryTableStats keyStats,
Expand All @@ -120,8 +120,8 @@ public DataFileMeta(
this(
fileName,
fileSize,
totalRowCount,
addRowCount,
rowCount,
deleteRowCount,
minKey,
maxKey,
keyStats,
Expand All @@ -137,8 +137,8 @@ public DataFileMeta(
public DataFileMeta(
String fileName,
long fileSize,
long totalRowCount,
@Nullable Long addRowCount,
long rowCount,
@Nullable Long deleteRowCount,
BinaryRow minKey,
BinaryRow maxKey,
BinaryTableStats keyStats,
Expand All @@ -152,8 +152,8 @@ public DataFileMeta(
this.fileName = fileName;
this.fileSize = fileSize;

this.totalRowCount = totalRowCount;
this.addRowCount = addRowCount;
this.rowCount = rowCount;
this.deleteRowCount = deleteRowCount;

this.minKey = minKey;
this.maxKey = maxKey;
Expand All @@ -176,16 +176,16 @@ public long fileSize() {
return fileSize;
}

public long totalRowCount() {
return totalRowCount;
public long rowCount() {
return rowCount;
}

public Optional<Long> addRowCount() {
return Optional.ofNullable(addRowCount);
return Optional.ofNullable(deleteRowCount).map(c -> rowCount - c);
}

public Optional<Long> deleteRowCount() {
return Optional.ofNullable(addRowCount).map(c -> totalRowCount - c);
return Optional.ofNullable(deleteRowCount);
}

public BinaryRow minKey() {
Expand Down Expand Up @@ -262,8 +262,8 @@ public DataFileMeta upgrade(int newLevel) {
return new DataFileMeta(
fileName,
fileSize,
totalRowCount,
addRowCount,
rowCount,
deleteRowCount,
minKey,
maxKey,
keyStats,
Expand All @@ -287,8 +287,8 @@ public DataFileMeta copy(List<String> newExtraFiles) {
return new DataFileMeta(
fileName,
fileSize,
totalRowCount,
addRowCount,
rowCount,
deleteRowCount,
minKey,
maxKey,
keyStats,
Expand All @@ -312,8 +312,8 @@ public boolean equals(Object o) {
DataFileMeta that = (DataFileMeta) o;
return Objects.equals(fileName, that.fileName)
&& fileSize == that.fileSize
&& totalRowCount == that.totalRowCount
&& Objects.equals(addRowCount, that.addRowCount)
&& rowCount == that.rowCount
&& Objects.equals(deleteRowCount, that.deleteRowCount)
&& Objects.equals(minKey, that.minKey)
&& Objects.equals(maxKey, that.maxKey)
&& Objects.equals(keyStats, that.keyStats)
Expand All @@ -331,9 +331,9 @@ public int hashCode() {
return Objects.hash(
fileName,
fileSize,
totalRowCount,
addRowCount,
minKey,
rowCount,
deleteRowCount,
deleteRowCount,
maxKey,
keyStats,
valueStats,
Expand All @@ -348,14 +348,14 @@ public int hashCode() {
@Override
public String toString() {
return String.format(
"{fileName: %s, fileSize: %d, totalRowCount: %d, addRowCount: %d, "
"{fileName: %s, fileSize: %d, rowCount: %d, deleteRowCount: %d, "
+ "minKey: %s, maxKey: %s, keyStats: %s, valueStats: %s, "
+ "minSequenceNumber: %d, maxSequenceNumber: %d, "
+ "schemaId: %d, level: %d, extraFiles: %s, creationTime: %s}",
fileName,
fileSize,
totalRowCount,
addRowCount,
rowCount,
deleteRowCount,
minKey,
maxKey,
keyStats,
Expand Down Expand Up @@ -383,7 +383,7 @@ public static RowType schema() {
fields.add(new DataField(10, "_LEVEL", new IntType(false)));
fields.add(new DataField(11, "_EXTRA_FILES", new ArrayType(false, newStringType(false))));
fields.add(new DataField(12, "_CREATION_TIME", DataTypes.TIMESTAMP_MILLIS()));
fields.add(new DataField(13, "_ADD_ROW_COUNT", new BigIntType(true)));
fields.add(new DataField(13, "_DELETE_ROW_COUNT", new BigIntType(true)));
return new RowType(fields);
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -43,7 +43,7 @@ public InternalRow toRow(DataFileMeta meta) {
return GenericRow.of(
BinaryString.fromString(meta.fileName()),
meta.fileSize(),
meta.totalRowCount(),
meta.rowCount(),
serializeBinaryRow(meta.minKey()),
serializeBinaryRow(meta.maxKey()),
meta.keyStats().toRow(),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -67,7 +67,7 @@ public class KeyValueDataFileWriter
private InternalRow maxKey = null;
private long minSeqNumber = Long.MAX_VALUE;
private long maxSeqNumber = Long.MIN_VALUE;
private long addRecordCount = 0;
private long deleteRecordCount = 0;

public KeyValueDataFileWriter(
FileIO fileIO,
Expand Down Expand Up @@ -112,8 +112,8 @@ public void write(KeyValue kv) throws IOException {
updateMinSeqNumber(kv);
updateMaxSeqNumber(kv);

if (kv.valueKind().isAdd()) {
addRecordCount++;
if (kv.valueKind().isRetract()) {
deleteRecordCount++;
}

if (LOG.isDebugEnabled()) {
Expand Down Expand Up @@ -160,7 +160,7 @@ public DataFileMeta result() throws IOException {
path.getName(),
fileIO.getFileSize(path),
recordCount(),
addRecordCount,
deleteRecordCount,
minKey,
keySerializer.toBinaryRow(maxKey).copy(),
keyStats,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -167,7 +167,7 @@ private LookupFile createLookupFile(DataFileMeta file) throws IOException {
throw new IOException("Can not create new file: " + localFile);
}
LookupStoreWriter kvWriter =
lookupStoreFactory.createWriter(localFile, bfGenerator.apply(file.totalRowCount()));
lookupStoreFactory.createWriter(localFile, bfGenerator.apply(file.rowCount()));
LookupStoreFactory.Context context;
try (RecordReader<KeyValue> reader = fileReaderFactory.apply(file)) {
KeyValue kv;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -83,10 +83,10 @@ protected boolean filterByStats(ManifestEntry entry) {
fieldStatsConverters.getOrCreate(entry.file().schemaId());
BinaryTableStats stats = entry.file().valueStats();
return filter.test(
entry.file().totalRowCount(),
entry.file().rowCount(),
serializer.evolution(stats.minValues()),
serializer.evolution(stats.maxValues()),
serializer.evolution(stats.nullCounts(), entry.file().totalRowCount()));
serializer.evolution(stats.nullCounts(), entry.file().rowCount()));
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -98,10 +98,10 @@ protected boolean filterByStats(ManifestEntry entry) {
fieldKeyStatsConverters.getOrCreate(entry.file().schemaId());
BinaryTableStats stats = entry.file().keyStats();
return keyFilter.test(
entry.file().totalRowCount(),
entry.file().rowCount(),
serializer.evolution(stats.minValues()),
serializer.evolution(stats.maxValues()),
serializer.evolution(stats.nullCounts(), entry.file().totalRowCount()));
serializer.evolution(stats.nullCounts(), entry.file().rowCount()));
}

/** Note: Keep this thread-safe. */
Expand Down Expand Up @@ -142,10 +142,10 @@ private boolean filterByValueFilter(ManifestEntry entry) {
fieldValueStatsConverters.getOrCreate(entry.file().schemaId());
BinaryTableStats stats = entry.file().valueStats();
return valueFilter.test(
entry.file().totalRowCount(),
entry.file().rowCount(),
serializer.evolution(stats.minValues()),
serializer.evolution(stats.maxValues()),
serializer.evolution(stats.nullCounts(), entry.file().totalRowCount()));
serializer.evolution(stats.nullCounts(), entry.file().rowCount()));
}

private static boolean noOverlapping(List<ManifestEntry> entries) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -126,7 +126,7 @@ protected static Map<BinaryRow, Set<Integer>> changedPartBuckets(
}

private long getRowCounts(List<ManifestEntry> files) {
return files.stream().mapToLong(file -> file.file().totalRowCount()).sum();
return files.stream().mapToLong(file -> file.file().rowCount()).sum();
}

@VisibleForTesting
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -101,7 +101,7 @@ public OptionalLong getLatestFileCreationEpochMillis() {
public long rowCount() {
long rowCount = 0;
for (DataFileMeta file : dataFiles) {
rowCount += file.totalRowCount();
rowCount += file.rowCount();
}
return rowCount;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -467,6 +467,6 @@ private RawFile makeRawTableFile(String bucketPath, DataFileMeta meta) {
.toString()
.toLowerCase()),
meta.schemaId(),
meta.totalRowCount());
meta.rowCount());
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -391,7 +391,7 @@ private LazyGenericRow toRow(
dataFileMeta.fileName())),
dataFileMeta::schemaId,
dataFileMeta::level,
dataFileMeta::totalRowCount,
dataFileMeta::rowCount,
dataFileMeta::fileSize,
() ->
dataFileMeta.minKey().getFieldCount() <= 0
Expand Down Expand Up @@ -447,7 +447,7 @@ private void initialize() {
InternalRow min = serializer.evolution(tableStats.minValues());
InternalRow max = serializer.evolution(tableStats.maxValues());
InternalArray nullCounts =
serializer.evolution(tableStats.nullCounts(), file.totalRowCount());
serializer.evolution(tableStats.nullCounts(), file.rowCount());
lazyNullValueCounts = new TreeMap<>();
lazyLowerValueBounds = new TreeMap<>();
lazyUpperValueBounds = new TreeMap<>();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -254,7 +254,7 @@ private LazyGenericRow toRow(
Supplier<Object>[] fields =
new Supplier[] {
() -> partitionId,
dataFileMeta::totalRowCount,
dataFileMeta::rowCount,
dataFileMeta::fileSize,
dataFileMeta::creationTimeEpochMillis
};
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -178,7 +178,7 @@ private DataFileMeta newFile(long fileSize) {
UUID.randomUUID().toString(),
fileSize,
1,
1L,
0L,
row(0),
row(0),
newTableStats(0, 1),
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -97,7 +97,7 @@ public void compactionTaskTest() throws Exception {
// one file is loaded from delta
List<DataFileMeta> last = new ArrayList<>(compactionCoordinator.listRestoredFiles());
assertThat(last.size()).isEqualTo(1);
assertThat(last.get(0).totalRowCount()).isEqualTo(11);
assertThat(last.get(0).rowCount()).isEqualTo(11);
}

@Test
Expand All @@ -114,7 +114,7 @@ public void testCompactionLot() throws Exception {
assertThat(compactionCoordinator.scan()).isTrue();
assertThat(
compactionCoordinator.listRestoredFiles().stream()
.map(DataFileMeta::totalRowCount)
.map(DataFileMeta::rowCount)
.reduce(Long::sum)
.get())
.isEqualTo(count);
Expand Down
Loading

0 comments on commit c051721

Please sign in to comment.