From 6ef78aff8499be2fa7d0df3e98bc6b42fff5e856 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Wed, 18 Dec 2024 16:15:33 +0800 Subject: [PATCH 01/12] add test --- .../session/it/IoTDBSessionSimpleIT.java | 76 +++++++++++++++++++ .../queryengine/plan/parser/ASTVisitor.java | 3 +- .../db/utils/TimestampPrecisionUtils.java | 10 ++- .../conf/iotdb-system.properties.template | 2 +- 4 files changed, 86 insertions(+), 5 deletions(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java index 7cfbed9fabe0..f06bd2ba38f5 100644 --- a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java @@ -36,12 +36,17 @@ import org.apache.tsfile.common.conf.TSFileConfig; import org.apache.tsfile.common.constant.TsFileConstant; import org.apache.tsfile.enums.TSDataType; +import org.apache.tsfile.exception.write.WriteProcessException; +import org.apache.tsfile.file.metadata.IDeviceID; +import org.apache.tsfile.file.metadata.IDeviceID.Factory; import org.apache.tsfile.file.metadata.enums.CompressionType; import org.apache.tsfile.file.metadata.enums.TSEncoding; import org.apache.tsfile.read.common.Field; import org.apache.tsfile.read.common.RowRecord; import org.apache.tsfile.utils.Binary; import org.apache.tsfile.utils.BitMap; +import org.apache.tsfile.write.TsFileWriter; +import org.apache.tsfile.write.record.TSRecord; import org.apache.tsfile.write.record.Tablet; import org.apache.tsfile.write.schema.IMeasurementSchema; import org.apache.tsfile.write.schema.MeasurementSchema; @@ -55,6 +60,8 @@ import org.slf4j.Logger; import org.slf4j.LoggerFactory; +import java.io.File; +import java.io.IOException; import java.security.SecureRandom; import java.util.ArrayList; import java.util.Arrays; @@ -1859,4 +1866,73 @@ public void convertRecordsToTabletsTest() { e.printStackTrace(); } } + + @Test + @Category({LocalStandaloneIT.class, ClusterIT.class}) + public void insertMinMaxTimeTest() throws IoTDBConnectionException, StatementExecutionException { + EnvFactory.getEnv().cleanClusterEnvironment(); + EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(false); + EnvFactory.getEnv().initClusterEnvironment(); + try { + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + session.executeNonQueryStatement( + String.format("INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MIN_VALUE)); + session.executeNonQueryStatement( + String.format("INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MAX_VALUE)); + + SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + RowRecord record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getTimestamp()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getTimestamp()); + assertFalse(dataSet.hasNext()); + + session.executeNonQueryStatement("FLUSH"); + dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getTimestamp()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getTimestamp()); + assertFalse(dataSet.hasNext()); + } + } finally { + EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(true); + } + } + + @Test + @Category({LocalStandaloneIT.class, ClusterIT.class}) + public void loadMinMaxTimeTest() + throws IoTDBConnectionException, + StatementExecutionException, + IOException, + WriteProcessException { + File file = new File("target", "test.tsfile"); + try (TsFileWriter writer = new TsFileWriter(file)) { + IDeviceID deviceID = Factory.DEFAULT_FACTORY.create(new String[] {"root", "test", "d1"}); + writer.registerTimeseries(deviceID, new MeasurementSchema("s1", TSDataType.INT32)); + TSRecord record = new TSRecord(deviceID, Long.MIN_VALUE); + record.addPoint("s1", 1); + writer.writeRecord(record); + record.setTime(Long.MAX_VALUE); + } + + EnvFactory.getEnv().cleanClusterEnvironment(); + EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(false); + EnvFactory.getEnv().initClusterEnvironment(); + try { + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + session.executeNonQueryStatement("LOAD " + file.getAbsolutePath()); + + SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + RowRecord record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getTimestamp()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getTimestamp()); + assertFalse(dataSet.hasNext()); + } + } finally { + EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(true); + } + } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java index 60bfcb81cd0a..ebc84c33a45a 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java @@ -1995,12 +1995,13 @@ private long parseTimeValue(ConstantContext constant) { if (constant.INTEGER_LITERAL() != null) { try { if (constant.MINUS() != null) { - return -Long.parseLong(constant.INTEGER_LITERAL().getText()); + return Long.parseLong("-" + constant.INTEGER_LITERAL().getText()); } return Long.parseLong(constant.INTEGER_LITERAL().getText()); } catch (NumberFormatException e) { throw new SemanticException( String.format( + "Failed to parse the timestamp: " + e.getMessage() + "Current system timestamp precision is %s, " + "please check whether the timestamp %s is correct.", TIMESTAMP_PRECISION, constant.INTEGER_LITERAL().getText())); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java index 768d0030e96d..fb0604d0919d 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java @@ -73,8 +73,10 @@ public static void checkTimestampPrecision(long time) { if (time > 10_000_000_000_000L) { throw new SemanticException( String.format( + "The timestamp is unexpectedly large, you may forget to set the timestamp precision." + "Current system timestamp precision is %s, " - + "please check whether the timestamp %s is correct.", + + "please check whether the timestamp %s is correct." + + "If you insist to insert this timestamp, please set timestamp_precision_check_enabled=false and restart the server.", TIMESTAMP_PRECISION, time)); } break; @@ -82,8 +84,10 @@ public static void checkTimestampPrecision(long time) { if (time > 10_000_000_000_000_000L) { throw new SemanticException( String.format( - "Current system timestamp precision is %s, " - + "please check whether the timestamp %s is correct.", + "The timestamp is unexpectedly large, you may forget to set the timestamp precision." + + "Current system timestamp precision is %s, " + + "please check whether the timestamp %s is correct." + + "If you insist to insert this timestamp, please set timestamp_precision_check_enabled=false and restart the server.", TIMESTAMP_PRECISION, time)); } break; diff --git a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template index f370aebbfa6b..705fe85d7576 100644 --- a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template +++ b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template @@ -1070,7 +1070,7 @@ timestamp_precision=ms # When the timestamp precision check is enabled, the timestamps those are over 13 digits for ms precision, or over 16 digits for us precision are not allowed to be inserted. # For all precisions, ms, us and ns, the timestamps cannot exceed the range of [-9223372036854775808, 9223372036854775807], regardless of whether the check is enabled or not. -# effectiveMode: first_start +# effectiveMode: restart # Datatype: Boolean timestamp_precision_check_enabled=true From bffdbf235baf0a5dd2bdf95b3307489fec28ab6e Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Wed, 18 Dec 2024 19:18:30 +0800 Subject: [PATCH 02/12] fix insert/load with Long.MIN/MAX --- .../session/it/IoTDBSessionSimpleIT.java | 78 ++++++++++++++----- .../apache/iotdb/db/conf/IoTDBDescriptor.java | 9 +++ .../plan/analyze/AnalyzeVisitor.java | 6 +- .../load/LoadTsFileTableSchemaCache.java | 1 + .../load/LoadTsFileTreeSchemaCache.java | 1 + .../queryengine/plan/parser/ASTVisitor.java | 8 +- .../AbstractCompactionEstimator.java | 3 + .../dataregion/memtable/AbstractMemTable.java | 4 +- .../dataregion/tsfile/TsFileResource.java | 8 +- .../timeindex/ArrayDeviceTimeIndex.java | 9 ++- .../dataregion/utils/TsFileResourceUtils.java | 18 ----- .../load/splitter/AlignedChunkData.java | 14 +++- .../BatchedAlignedValueChunkData.java | 6 +- .../load/splitter/NonAlignedChunkData.java | 8 +- .../load/splitter/TsFileSplitter.java | 7 ++ .../db/utils/TimestampPrecisionUtils.java | 16 ++-- .../resources/conf/iotdb-system.properties | 16 ++++ 17 files changed, 146 insertions(+), 66 deletions(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java index f06bd2ba38f5..2ae4498dd4b8 100644 --- a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java @@ -79,6 +79,7 @@ import static org.junit.Assert.assertTrue; import static org.junit.Assert.fail; +@SuppressWarnings("ThrowFromFinallyBlock") @RunWith(IoTDBTestRunner.class) public class IoTDBSessionSimpleIT { @@ -1870,15 +1871,24 @@ public void convertRecordsToTabletsTest() { @Test @Category({LocalStandaloneIT.class, ClusterIT.class}) public void insertMinMaxTimeTest() throws IoTDBConnectionException, StatementExecutionException { - EnvFactory.getEnv().cleanClusterEnvironment(); - EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(false); - EnvFactory.getEnv().initClusterEnvironment(); try { try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION \"timestamp_precision_check_enabled\"=\"false\""); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + session.executeNonQueryStatement( - String.format("INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MIN_VALUE)); + String.format( + "INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MIN_VALUE)); session.executeNonQueryStatement( - String.format("INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MAX_VALUE)); + String.format( + "INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MAX_VALUE)); SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); RowRecord record = dataSet.next(); @@ -1896,7 +1906,17 @@ record = dataSet.next(); assertFalse(dataSet.hasNext()); } } finally { - EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(true); + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION \"timestamp_precision_check_enabled\"=\"true\""); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + } } } @@ -1909,30 +1929,46 @@ public void loadMinMaxTimeTest() WriteProcessException { File file = new File("target", "test.tsfile"); try (TsFileWriter writer = new TsFileWriter(file)) { - IDeviceID deviceID = Factory.DEFAULT_FACTORY.create(new String[] {"root", "test", "d1"}); + IDeviceID deviceID = Factory.DEFAULT_FACTORY.create("root.test.d1"); writer.registerTimeseries(deviceID, new MeasurementSchema("s1", TSDataType.INT32)); TSRecord record = new TSRecord(deviceID, Long.MIN_VALUE); record.addPoint("s1", 1); writer.writeRecord(record); record.setTime(Long.MAX_VALUE); + writer.writeRecord(record); } - EnvFactory.getEnv().cleanClusterEnvironment(); - EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(false); - EnvFactory.getEnv().initClusterEnvironment(); - try { - try (ISession session = EnvFactory.getEnv().getSessionConnection()) { - session.executeNonQueryStatement("LOAD " + file.getAbsolutePath()); - - SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); - RowRecord record = dataSet.next(); - assertEquals(Long.MIN_VALUE, record.getTimestamp()); - record = dataSet.next(); - assertEquals(Long.MAX_VALUE, record.getTimestamp()); - assertFalse(dataSet.hasNext()); + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION \"timestamp_precision_check_enabled\"=\"false\""); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } } + session.executeNonQueryStatement("LOAD \"" + file.getAbsolutePath() + "\""); + + SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + RowRecord record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getTimestamp()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getTimestamp()); + assertFalse(dataSet.hasNext()); } finally { - EnvFactory.getEnv().getConfig().getCommonConfig().setTimestampPrecisionCheckEnabled(true); + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION \"timestamp_precision_check_enabled\"=\"true\""); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + } + file.delete(); } } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/conf/IoTDBDescriptor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/conf/IoTDBDescriptor.java index fce4bb4e9333..4d0705f108a3 100755 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/conf/IoTDBDescriptor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/conf/IoTDBDescriptor.java @@ -2882,6 +2882,15 @@ public synchronized void loadHotModifiedProps(TrimProperties properties) BinaryAllocator.getInstance().close(true); } + commonDescriptor + .getConfig() + .setTimestampPrecisionCheckEnabled( + Boolean.parseBoolean( + properties.getProperty( + "timestamp_precision_check_enabled", + ConfigurationFileUtils.getConfigurationDefaultValue( + "timestamp_precision_check_enabled")))); + // update query_sample_throughput_bytes_per_sec loadQuerySampleThroughput(properties); // update trusted_uri_pattern diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/AnalyzeVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/AnalyzeVisitor.java index 224d22520a0a..7ca0b9279d13 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/AnalyzeVisitor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/AnalyzeVisitor.java @@ -2301,7 +2301,11 @@ public static Pair, Pair> getTimePart result.add(timePartitionSlot); // next init timePartitionSlot = new TTimePartitionSlot(endTime); - endTime = endTime + TimePartitionUtils.getTimePartitionInterval(); + // beware of overflow + endTime = + endTime + TimePartitionUtils.getTimePartitionInterval() > endTime + ? endTime + TimePartitionUtils.getTimePartitionInterval() + : Long.MAX_VALUE; } else { index++; if (index < size) { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java index 127b619f06c6..ac715923945d 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java @@ -134,6 +134,7 @@ public void autoCreateAndVerify(IDeviceID device) { private boolean isDeviceDeletedByMods(IDeviceID device) throws IllegalPathException { return currentTimeIndex != null + && currentTimeIndex.checkDeviceIdExist(device) && ModificationUtils.isAllDeletedByMods( currentModifications, device, diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java index f982a03affd8..c6e215e67bd6 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java @@ -189,6 +189,7 @@ public void setCurrentModificationsAndTimeIndex( public boolean isDeviceDeletedByMods(IDeviceID device) throws IllegalPathException { return currentTimeIndex != null + && currentTimeIndex.checkDeviceIdExist(device) && ModificationUtils.isAllDeletedByMods( currentModifications, device, diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java index ebc84c33a45a..7a274b380462 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/parser/ASTVisitor.java @@ -2001,10 +2001,12 @@ private long parseTimeValue(ConstantContext constant) { } catch (NumberFormatException e) { throw new SemanticException( String.format( - "Failed to parse the timestamp: " + e.getMessage() + - "Current system timestamp precision is %s, " + "Failed to parse the timestamp: " + + e.getMessage() + + "Current system timestamp precision is %s, " + "please check whether the timestamp %s is correct.", - TIMESTAMP_PRECISION, constant.INTEGER_LITERAL().getText())); + TIMESTAMP_PRECISION, + constant.INTEGER_LITERAL().getText())); } } else if (constant.dateExpression() != null) { return parseDateExpression(constant.dateExpression(), CommonDateTimeUtils.currentTime()); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java index 4ec74c57d63e..fb9d825a5918 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java @@ -140,6 +140,9 @@ protected int calculatingMaxOverlapFileNumInSubCompactionTask(List getMaxTime() { Map latestTimeForEachDevice = new HashMap<>(); for (Entry entry : memTableMap.entrySet()) { - // When insert null values in to IWritableMemChunkGroup, the maxTime will not be updated. - // In this scenario, the maxTime will be Long.MIN_VALUE. We shouldn't return this device. long maxTime = entry.getValue().getMaxTime(); - if (maxTime != Long.MIN_VALUE) { + if (entry.getValue().count() > 0) { latestTimeForEachDevice.put(entry.getKey(), maxTime); } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java index ed681abe8ad9..366eeef0ee61 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java @@ -576,7 +576,9 @@ public long getTsFileSize() { public long getStartTime(IDeviceID deviceId) { try { - return deviceId == null ? getFileStartTime() : timeIndex.getStartTime(deviceId); + return deviceId == null || !timeIndex.checkDeviceIdExist(deviceId) + ? getFileStartTime() + : timeIndex.getStartTime(deviceId); } catch (Exception e) { LOGGER.error( "meet error when getStartTime of {} in file {}", deviceId, file.getAbsolutePath(), e); @@ -590,7 +592,9 @@ public long getStartTime(IDeviceID deviceId) { /** open file's end time is Long.MIN_VALUE */ public long getEndTime(IDeviceID deviceId) { try { - return deviceId == null ? getFileEndTime() : timeIndex.getEndTime(deviceId); + return deviceId == null || !timeIndex.checkDeviceIdExist(deviceId) + ? getFileEndTime() + : timeIndex.getEndTime(deviceId); } catch (Exception e) { LOGGER.error( "meet error when getEndTime of {} in file {}", deviceId, file.getAbsolutePath(), e); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java index 31fd9d385e62..8b99caca8c54 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java @@ -43,6 +43,7 @@ import java.util.HashSet; import java.util.Map; import java.util.Map.Entry; +import java.util.NoSuchElementException; import java.util.Set; import java.util.concurrent.ConcurrentHashMap; @@ -318,9 +319,9 @@ private long getTimePartitionWithCheck() throws PartitionViolationException { @Override public void updateStartTime(IDeviceID deviceId, long time) { + int index = getDeviceIndex(deviceId); long startTime = getStartTime(deviceId); if (time < startTime) { - int index = getDeviceIndex(deviceId); startTimes[index] = time; } minStartTime = Math.min(minStartTime, time); @@ -328,9 +329,9 @@ public void updateStartTime(IDeviceID deviceId, long time) { @Override public void updateEndTime(IDeviceID deviceId, long time) { + int index = getDeviceIndex(deviceId); long endTime = getEndTime(deviceId); if (time > endTime) { - int index = getDeviceIndex(deviceId); endTimes[index] = time; } maxEndTime = Math.max(maxEndTime, time); @@ -353,7 +354,7 @@ public void putEndTime(IDeviceID deviceId, long time) { @Override public long getStartTime(IDeviceID deviceId) { if (!deviceToIndex.containsKey(deviceId)) { - return Long.MAX_VALUE; + throw new NoSuchElementException(deviceId.toString()); } return startTimes[deviceToIndex.get(deviceId)]; } @@ -361,7 +362,7 @@ public long getStartTime(IDeviceID deviceId) { @Override public long getEndTime(IDeviceID deviceId) { if (!deviceToIndex.containsKey(deviceId)) { - return Long.MIN_VALUE; + throw new NoSuchElementException(deviceId.toString()); } return endTimes[deviceToIndex.get(deviceId)]; } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java index d8bc61db50a2..679d5119b9ec 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java @@ -94,24 +94,6 @@ public static boolean validateTsFileResourceCorrectness(TsFileResource resource) for (IDeviceID device : devices) { long startTime = timeIndex.getStartTime(device); long endTime = timeIndex.getEndTime(device); - if (startTime == Long.MAX_VALUE) { - logger.error( - "{} {} the start time of {} is {}", - resource.getTsFilePath(), - VALIDATE_FAILED, - device, - Long.MAX_VALUE); - return false; - } - if (endTime == Long.MIN_VALUE) { - logger.error( - "{} {} the end time of {} is {}", - resource.getTsFilePath(), - VALIDATE_FAILED, - device, - Long.MIN_VALUE); - return false; - } if (startTime > endTime) { logger.error( "{} {} the start time of {} is greater than end time", diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java index 585421e3afe8..225d87a86bc4 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java @@ -215,13 +215,17 @@ public void writeDecodePage(final long[] times, final Object[] values, final int pageNumbers.set(pageNumbers.size() - 1, pageNumbers.get(pageNumbers.size() - 1) + 1); satisfiedLengthQueue.offer(satisfiedLength); final long startTime = timePartitionSlot.getStartTime(); - final long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + // beware of overflow + long endTime = startTime + TimePartitionUtils.getTimePartitionInterval() - 1; + if (endTime <= startTime) { + endTime = Long.MAX_VALUE; + } // serialize needDecode==true dataSize += ReadWriteIOUtils.write(true, stream); dataSize += ReadWriteIOUtils.write(satisfiedLength, stream); for (final long time : times) { - if (time >= endTime) { + if (time > endTime) { break; } if (time >= startTime) { @@ -235,7 +239,11 @@ public void writeDecodeValuePage( throws IOException { pageNumbers.set(pageNumbers.size() - 1, pageNumbers.get(pageNumbers.size() - 1) + 1); final long startTime = timePartitionSlot.getStartTime(); - final long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + // beware of overflow + long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + if (endTime <= startTime) { + endTime = Long.MAX_VALUE; + } final int satisfiedLength = satisfiedLengthQueue.poll(); // serialize needDecode==true dataSize += ReadWriteIOUtils.write(true, stream); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java index a7b5a840ba66..d79fdeb09e05 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java @@ -70,7 +70,11 @@ public void writeDecodeValuePage(long[] times, TsPrimitiveType[] values, TSDataT throws IOException { pageNumbers.set(pageNumbers.size() - 1, pageNumbers.get(pageNumbers.size() - 1) + 1); final long startTime = timePartitionSlot.getStartTime(); - final long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + // beware of overflow + long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + if (endTime <= startTime) { + endTime = Long.MAX_VALUE; + } final int satisfiedLength = satisfiedLengthQueue.poll(); // serialize needDecode==true dataSize += ReadWriteIOUtils.write(true, stream); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/NonAlignedChunkData.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/NonAlignedChunkData.java index f2d17ab95fc3..ccf7970a1db2 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/NonAlignedChunkData.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/NonAlignedChunkData.java @@ -163,12 +163,16 @@ public void writeDecodePage(final long[] times, final Object[] values, final int throws IOException { pageNumber += 1; final long startTime = timePartitionSlot.getStartTime(); - final long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + // beware of overflow + long endTime = startTime + TimePartitionUtils.getTimePartitionInterval() - 1; + if (endTime <= startTime) { + endTime = Long.MAX_VALUE; + } dataSize += ReadWriteIOUtils.write(true, stream); dataSize += ReadWriteIOUtils.write(satisfiedLength, stream); for (int i = 0; i < times.length; i++) { - if (times[i] >= endTime) { + if (times[i] > endTime) { break; } if (times[i] >= startTime) { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/TsFileSplitter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/TsFileSplitter.java index 5c86082ce8c9..387007661431 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/TsFileSplitter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/TsFileSplitter.java @@ -249,6 +249,10 @@ private void decodeAndWriteTimeChunkOrNonAlignedChunk( int satisfiedLength = 0; long endTime = timePartitionSlot.getStartTime() + TimePartitionUtils.getTimePartitionInterval(); + // beware of overflow + if (endTime <= timePartitionSlot.getStartTime()) { + endTime = Long.MAX_VALUE; + } for (int i = 0; i < times.length; i++) { if (times[i] >= endTime) { chunkData.writeDecodePage(times, values, satisfiedLength); @@ -264,6 +268,9 @@ private void decodeAndWriteTimeChunkOrNonAlignedChunk( satisfiedLength = 0; endTime = timePartitionSlot.getStartTime() + TimePartitionUtils.getTimePartitionInterval(); + if (endTime <= timePartitionSlot.getStartTime()) { + endTime = Long.MAX_VALUE; + } chunkData = ChunkData.createChunkData(isAligned, curDevice, header, timePartitionSlot); } satisfiedLength += 1; diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java index fb0604d0919d..5b0a642a0eba 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java @@ -73,10 +73,10 @@ public static void checkTimestampPrecision(long time) { if (time > 10_000_000_000_000L) { throw new SemanticException( String.format( - "The timestamp is unexpectedly large, you may forget to set the timestamp precision." + - "Current system timestamp precision is %s, " - + "please check whether the timestamp %s is correct." + - "If you insist to insert this timestamp, please set timestamp_precision_check_enabled=false and restart the server.", + "The timestamp is unexpectedly large, you may forget to set the timestamp precision." + + "Current system timestamp precision is %s, " + + "please check whether the timestamp %s is correct." + + "If you insist to insert this timestamp, please set timestamp_precision_check_enabled=false and restart the server.", TIMESTAMP_PRECISION, time)); } break; @@ -84,10 +84,10 @@ public static void checkTimestampPrecision(long time) { if (time > 10_000_000_000_000_000L) { throw new SemanticException( String.format( - "The timestamp is unexpectedly large, you may forget to set the timestamp precision." + - "Current system timestamp precision is %s, " - + "please check whether the timestamp %s is correct." + - "If you insist to insert this timestamp, please set timestamp_precision_check_enabled=false and restart the server.", + "The timestamp is unexpectedly large, you may forget to set the timestamp precision." + + "Current system timestamp precision is %s, " + + "please check whether the timestamp %s is correct." + + "If you insist to insert this timestamp, please set timestamp_precision_check_enabled=false and restart the server.", TIMESTAMP_PRECISION, time)); } break; diff --git a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties index c51f948ad250..db2835acfc52 100644 --- a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties +++ b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties @@ -70,3 +70,19 @@ cn_metric_prometheus_reporter_port=9091 # dn_metric_reporter_list= dn_metric_prometheus_reporter_port=9092 +#Wed Dec 18 17:50:40 HKT 2024 +#timestamp_precision_check_enabled=false +#Wed Dec 18 18:05:55 HKT 2024 +#timestamp_precision_check_enabled=true +#Wed Dec 18 18:17:25 HKT 2024 +#timestamp_precision_check_enabled=false +#Wed Dec 18 18:17:25 HKT 2024 +#timestamp_precision_check_enabled=true +#Wed Dec 18 18:18:03 HKT 2024 +#timestamp_precision_check_enabled=false +#Wed Dec 18 18:19:27 HKT 2024 +#timestamp_precision_check_enabled=true +#Wed Dec 18 18:19:55 HKT 2024 +#timestamp_precision_check_enabled=false +#Wed Dec 18 18:55:14 HKT 2024 +timestamp_precision_check_enabled=true From 2445092ecd8c24cb2ca170f2752de79aa638fbc7 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Thu, 19 Dec 2024 11:24:41 +0800 Subject: [PATCH 03/12] Fix tests --- .../session/it/IoTDBSessionSimpleIT.java | 18 +++++++++------ .../plan/node/load/LoadSingleTsFileNode.java | 11 ++++++---- .../fast/SeriesCompactionExecutor.java | 3 +++ .../SingleSeriesCompactionExecutor.java | 22 ++++++++++--------- .../dataregion/tsfile/TsFileResource.java | 16 +++++++------- .../db/tools/utils/TsFileValidationScan.java | 1 + .../db/utils/TimestampPrecisionUtils.java | 4 ++-- .../TsFileValidationCorrectnessTests.java | 14 +++++++----- .../utils/CompactionCheckerUtils.java | 6 +++++ 9 files changed, 58 insertions(+), 37 deletions(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java index 2ae4498dd4b8..f7fa3e3345b1 100644 --- a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java @@ -344,7 +344,7 @@ public void insertByObjAndNotInferTypeTest() { expected.add(TSDataType.TEXT.name()); Set actual = new HashSet<>(); - SessionDataSet dataSet = session.executeQueryStatement("show timeseries root.**"); + SessionDataSet dataSet = session.executeQueryStatement("show timeseries root.sg1.**"); while (dataSet.hasNext()) { actual.add(dataSet.next().getFields().get(3).getStringValue()); } @@ -1885,12 +1885,15 @@ public void insertMinMaxTimeTest() throws IoTDBConnectionException, StatementExe session.executeNonQueryStatement( String.format( - "INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MIN_VALUE)); + "INSERT INTO root.testInsertMinMax.d1(timestamp, s1) VALUES (%d, 1)", + Long.MIN_VALUE)); session.executeNonQueryStatement( String.format( - "INSERT INTO root.test.d1(timestamp, s1) VALUES (%d, 1)", Long.MAX_VALUE)); + "INSERT INTO root.testInsertMinMax.d1(timestamp, s1) VALUES (%d, 1)", + Long.MAX_VALUE)); - SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + SessionDataSet dataSet = + session.executeQueryStatement("SELECT * FROM root.testInsertMinMax.d1"); RowRecord record = dataSet.next(); assertEquals(Long.MIN_VALUE, record.getTimestamp()); record = dataSet.next(); @@ -1898,7 +1901,7 @@ record = dataSet.next(); assertFalse(dataSet.hasNext()); session.executeNonQueryStatement("FLUSH"); - dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + dataSet = session.executeQueryStatement("SELECT * FROM root.testInsertMinMax.d1"); record = dataSet.next(); assertEquals(Long.MIN_VALUE, record.getTimestamp()); record = dataSet.next(); @@ -1929,7 +1932,7 @@ public void loadMinMaxTimeTest() WriteProcessException { File file = new File("target", "test.tsfile"); try (TsFileWriter writer = new TsFileWriter(file)) { - IDeviceID deviceID = Factory.DEFAULT_FACTORY.create("root.test.d1"); + IDeviceID deviceID = Factory.DEFAULT_FACTORY.create("root.testLoadMinMax.d1"); writer.registerTimeseries(deviceID, new MeasurementSchema("s1", TSDataType.INT32)); TSRecord record = new TSRecord(deviceID, Long.MIN_VALUE); record.addPoint("s1", 1); @@ -1950,7 +1953,8 @@ public void loadMinMaxTimeTest() } session.executeNonQueryStatement("LOAD \"" + file.getAbsolutePath() + "\""); - SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM root.test.d1"); + SessionDataSet dataSet = + session.executeQueryStatement("SELECT * FROM root.testLoadMinMax.d1"); RowRecord record = dataSet.next(); assertEquals(Long.MIN_VALUE, record.getTimestamp()); record = dataSet.next(); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java index 9a6314fbf293..c9d7567ad9ee 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java @@ -93,10 +93,13 @@ public boolean needDecodeTsFile( .getDevices() .forEach( o -> { - slotList.add( - new Pair<>(o, TimePartitionUtils.getTimePartitionSlot(resource.getStartTime(o)))); - slotList.add( - new Pair<>(o, TimePartitionUtils.getTimePartitionSlot(resource.getEndTime(o)))); + if (!resource.definitelyNotContains(o)) { + slotList.add( + new Pair<>( + o, TimePartitionUtils.getTimePartitionSlot(resource.getStartTime(o)))); + slotList.add( + new Pair<>(o, TimePartitionUtils.getTimePartitionSlot(resource.getEndTime(o)))); + } }); if (slotList.isEmpty()) { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java index 0dc75857e041..d631a272e986 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java @@ -353,6 +353,9 @@ protected List findOverlapFiles(FileElement file) { List overlappedFiles = new ArrayList<>(); long endTime = file.resource.getEndTime(deviceId); for (FileElement fileElement : fileList) { + if (fileElement.resource.definitelyNotContains(deviceId)) { + continue; + } if (fileElement.resource.getStartTime(deviceId) <= endTime) { if (!fileElement.isSelected) { overlappedFiles.add(fileElement); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java index a8bdcf985b99..68c176591ca4 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java @@ -57,9 +57,9 @@ public class SingleSeriesCompactionExecutor { private Chunk cachedChunk; private ChunkMetadata cachedChunkMetadata; // record the min time and max time to update the target resource - private long minStartTimestamp = Long.MAX_VALUE; - private long maxEndTimestamp = Long.MIN_VALUE; - private long lastWriteTimestamp = Long.MIN_VALUE; + private Long minStartTimestamp = null; + private Long maxEndTimestamp = null; + private Long lastWriteTimestamp = null; private long pointCountInChunkWriter = 0; private final CompactionTaskSummary summary; @@ -158,8 +158,10 @@ public void execute() throws IOException { flushChunkWriter(); } fileWriter.checkMetadataSizeAndMayFlush(); - targetResource.updateStartTime(device, minStartTimestamp); - targetResource.updateEndTime(device, maxEndTimestamp); + if (minStartTimestamp != null) { + targetResource.updateStartTime(device, minStartTimestamp); + targetResource.updateEndTime(device, maxEndTimestamp); + } } private void constructChunkWriterFromReadChunk(Chunk chunk) { @@ -269,10 +271,10 @@ private void writeChunkIntoChunkWriter(Chunk chunk) throws IOException { TimeValuePair timeValuePair = batchIterator.nextTimeValuePair(); checkAndUpdatePreviousTimestamp(timeValuePair.getTimestamp()); writeTimeAndValueToChunkWriter(timeValuePair); - if (timeValuePair.getTimestamp() > maxEndTimestamp) { + if (maxEndTimestamp == null || timeValuePair.getTimestamp() > maxEndTimestamp) { maxEndTimestamp = timeValuePair.getTimestamp(); } - if (timeValuePair.getTimestamp() < minStartTimestamp) { + if (minStartTimestamp == null || timeValuePair.getTimestamp() < minStartTimestamp) { minStartTimestamp = timeValuePair.getTimestamp(); } } @@ -336,10 +338,10 @@ private void writeTimeAndValueToChunkWriter(TimeValuePair timeValuePair) { private void flushChunkToFileWriter(Chunk chunk, ChunkMetadata chunkMetadata) throws IOException { checkAndUpdatePreviousTimestamp(chunkMetadata.getStartTime()); - if (chunkMetadata.getStartTime() < minStartTimestamp) { + if (minStartTimestamp == null || chunkMetadata.getStartTime() < minStartTimestamp) { minStartTimestamp = chunkMetadata.getStartTime(); } - if (chunkMetadata.getEndTime() > maxEndTimestamp) { + if (maxEndTimestamp == null || chunkMetadata.getEndTime() > maxEndTimestamp) { maxEndTimestamp = chunkMetadata.getEndTime(); } fileWriter.writeChunk(chunk, chunkMetadata); @@ -368,7 +370,7 @@ private void flushChunkWriter() throws IOException { } private void checkAndUpdatePreviousTimestamp(long currentWritingTimestamp) { - if (currentWritingTimestamp <= lastWriteTimestamp) { + if (lastWriteTimestamp != null && currentWritingTimestamp <= lastWriteTimestamp) { throw new CompactionLastTimeCheckFailedException( device, measurement, currentWritingTimestamp, lastWriteTimestamp); } else { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java index 366eeef0ee61..e1b87ec0e36b 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java @@ -576,9 +576,7 @@ public long getTsFileSize() { public long getStartTime(IDeviceID deviceId) { try { - return deviceId == null || !timeIndex.checkDeviceIdExist(deviceId) - ? getFileStartTime() - : timeIndex.getStartTime(deviceId); + return deviceId == null ? getFileStartTime() : timeIndex.getStartTime(deviceId); } catch (Exception e) { LOGGER.error( "meet error when getStartTime of {} in file {}", deviceId, file.getAbsolutePath(), e); @@ -592,9 +590,7 @@ public long getStartTime(IDeviceID deviceId) { /** open file's end time is Long.MIN_VALUE */ public long getEndTime(IDeviceID deviceId) { try { - return deviceId == null || !timeIndex.checkDeviceIdExist(deviceId) - ? getFileEndTime() - : timeIndex.getEndTime(deviceId); + return deviceId == null ? getFileEndTime() : timeIndex.getEndTime(deviceId); } catch (Exception e) { LOGGER.error( "meet error when getEndTime of {} in file {}", deviceId, file.getAbsolutePath(), e); @@ -607,7 +603,7 @@ public long getEndTime(IDeviceID deviceId) { // cannot use FileTimeIndex public long getOrderTimeForSeq(IDeviceID deviceId, boolean ascending) { - if (timeIndex instanceof ArrayDeviceTimeIndex) { + if (timeIndex instanceof ArrayDeviceTimeIndex && !definitelyNotContains(deviceId)) { return ascending ? getStartTime(deviceId) : getEndTime(deviceId); } else { return ascending ? Long.MIN_VALUE : Long.MAX_VALUE; @@ -616,7 +612,11 @@ public long getOrderTimeForSeq(IDeviceID deviceId, boolean ascending) { // can use FileTimeIndex public long getOrderTimeForUnseq(IDeviceID deviceId, boolean ascending) { - return ascending ? getStartTime(deviceId) : getEndTime(deviceId); + if (!definitelyNotContains(deviceId)) { + return ascending ? getStartTime(deviceId) : getEndTime(deviceId); + } else { + return ascending ? Long.MIN_VALUE : Long.MAX_VALUE; + } } @Override diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java index d4fd2356ea67..6756c88d329c 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java @@ -140,6 +140,7 @@ protected void onChunkGroup() throws IOException { fileNameLastTimePair = deviceEndTime.computeIfAbsent(currDeviceID, k -> new FileLastTimeInfo()); if (!Boolean.TRUE.equals(hasCheckedDeviceOverlap.getOrDefault(currDeviceID, false)) + && !resource.definitelyNotContains(currDeviceID) && resource.getStartTime(currDeviceID) <= fileNameLastTimePair.endTimeInLastFile) { // device overlap, find bad file recordDeviceOverlap(fileNameLastTimePair.lastFileName); diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java index 5b0a642a0eba..44c8f117469e 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/TimestampPrecisionUtils.java @@ -26,8 +26,6 @@ public class TimestampPrecisionUtils { public static String TIMESTAMP_PRECISION = CommonDescriptor.getInstance().getConfig().getTimestampPrecision(); - private static final boolean isTimestampPrecisionCheckEnabled = - CommonDescriptor.getInstance().getConfig().isTimestampPrecisionCheckEnabled(); @FunctionalInterface private interface ConvertFunction { @@ -65,6 +63,8 @@ public static long convertToCurrPrecision(long sourceTime, TimeUnit sourceUnit) /** check whether the input timestamp match the current system timestamp precision. */ public static void checkTimestampPrecision(long time) { + final boolean isTimestampPrecisionCheckEnabled = + CommonDescriptor.getInstance().getConfig().isTimestampPrecisionCheckEnabled(); if (!isTimestampPrecisionCheckEnabled) { return; } diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/TsFileValidationCorrectnessTests.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/TsFileValidationCorrectnessTests.java index f47dbc839e90..0fe2f5b31be9 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/TsFileValidationCorrectnessTests.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/TsFileValidationCorrectnessTests.java @@ -165,8 +165,9 @@ public void testAlignedTimestampHasOverlapBetweenPages() throws IOException { public void testAlignedTimestampTimeChunkOffsetEqualsMetadata() throws IOException { String path = dir + File.separator + "test6.tsfile"; TsFileResource tsFileResource = new TsFileResource(new File(path)); + IDeviceID deviceID; try (CompactionTestFileWriter writer = new CompactionTestFileWriter(tsFileResource)) { - writer.startChunkGroup("d1"); + deviceID = writer.startChunkGroup("d1"); VectorMeasurementSchema vectorMeasurementSchema = new VectorMeasurementSchema( "d1", new String[] {"s1"}, new TSDataType[] {TSDataType.INT32}); @@ -182,8 +183,8 @@ public void testAlignedTimestampTimeChunkOffsetEqualsMetadata() throws IOExcepti writer.endChunkGroup(); writer.endFile(); } - tsFileResource.updateStartTime(IDeviceID.Factory.DEFAULT_FACTORY.create("d1"), 1); - tsFileResource.updateEndTime(IDeviceID.Factory.DEFAULT_FACTORY.create("d1"), 3); + tsFileResource.updateStartTime(deviceID, 1); + tsFileResource.updateEndTime(deviceID, 3); tsFileResource.serialize(); boolean success = TsFileValidator.getInstance().validateTsFile(tsFileResource); Assert.assertTrue(success); @@ -272,8 +273,9 @@ public void testNonAlignedTimestampHasOverlapBetweenPages() throws IOException { public void testNonAlignedTimestampTimeChunkOffsetEqualsMetadata() throws IOException { String path = dir + File.separator + "test11.tsfile"; TsFileResource tsFileResource = new TsFileResource(new File(path)); + IDeviceID deviceID; try (CompactionTestFileWriter writer = new CompactionTestFileWriter(tsFileResource)) { - writer.startChunkGroup("d1"); + deviceID = writer.startChunkGroup("d1"); VectorMeasurementSchema vectorMeasurementSchema = new VectorMeasurementSchema( "d1", new String[] {"s1"}, new TSDataType[] {TSDataType.INT32}); @@ -289,8 +291,8 @@ public void testNonAlignedTimestampTimeChunkOffsetEqualsMetadata() throws IOExce writer.endChunkGroup(); writer.endFile(); } - tsFileResource.updateStartTime(IDeviceID.Factory.DEFAULT_FACTORY.create("d1"), 1); - tsFileResource.updateEndTime(IDeviceID.Factory.DEFAULT_FACTORY.create("d1"), 3); + tsFileResource.updateStartTime(deviceID, 1); + tsFileResource.updateEndTime(deviceID, 3); tsFileResource.serialize(); boolean success = TsFileValidator.getInstance().validateTsFile(tsFileResource); Assert.assertTrue(success); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java index 77e8498c8587..13d3075be942 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java @@ -353,10 +353,16 @@ private static void checkTsFileResource( long[] statistics = deviceCountEntry.getValue(); long startTime = Long.MAX_VALUE; for (TsFileResource mergedFile : mergedFiles) { + if (mergedFile.definitelyNotContains(device)) { + continue; + } startTime = Math.min(startTime, mergedFile.getStartTime(device)); } long endTime = Long.MIN_VALUE; for (TsFileResource mergedFile : mergedFiles) { + if (mergedFile.definitelyNotContains(device)) { + continue; + } endTime = Math.max(endTime, mergedFile.getEndTime(device)); } assertEquals(statistics[0], startTime); From 8a1cef9eeb84fb6d91e091f7da048dda6bee6911 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Thu, 19 Dec 2024 12:10:18 +0800 Subject: [PATCH 04/12] fix tests --- .../compaction/execute/utils/MultiTsFileDeviceIterator.java | 6 ++++-- .../dataregion/wal/recover/file/TsFilePlanRedoer.java | 4 ++-- .../apache/iotdb/db/tools/utils/TsFileValidationScan.java | 2 +- 3 files changed, 7 insertions(+), 5 deletions(-) diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java index 26f20697e2da..1e153551ead9 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java @@ -425,7 +425,8 @@ private void applyModificationForAlignedChunkMetadataList( } IDeviceID device = currentDevice.getLeft(); ModEntry ttlDeletion = null; - if (tsFileResource.getStartTime(device) < timeLowerBoundForCurrentDevice) { + if (!tsFileResource.definitelyNotContains(device) + && tsFileResource.getStartTime(device) < timeLowerBoundForCurrentDevice) { ttlDeletion = CompactionUtils.convertTtlToDeletion(device, timeLowerBoundForCurrentDevice); } @@ -662,7 +663,8 @@ public String nextSeries() throws IllegalPathException { Map> chunkMetadataListMap = chunkMetadataCacheMap.get(reader); ModEntry ttlDeletion = null; - if (resource.getStartTime(device) < timeLowerBoundForCurrentDevice) { + if (!resource.definitelyNotContains(device) + && resource.getStartTime(device) < timeLowerBoundForCurrentDevice) { ttlDeletion = new TreeDeletionEntry( new MeasurementPath(device, IoTDBConstant.ONE_LEVEL_PATH_WILDCARD), diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java index e3d560a01731..159382a84968 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java @@ -85,7 +85,7 @@ void redoInsert(InsertNode node) throws WriteProcessException { if (!node.hasValidMeasurements()) { return; } - if (tsFileResource != null) { + if (tsFileResource != null && !tsFileResource.definitelyNotContains(node.getDeviceID())) { // orders of insert node is guaranteed by storage engine, just check time in the file // the last chunk group may contain the same data with the logs, ignore such logs in seq file long lastEndTime = tsFileResource.getEndTime(node.getDeviceID()); @@ -127,7 +127,7 @@ void redoInsertRows(InsertRowsNode insertRowsNode) { if (!node.hasValidMeasurements()) { continue; } - if (tsFileResource != null) { + if (tsFileResource != null && !tsFileResource.definitelyNotContains(node.getDeviceID())) { // orders of insert node is guaranteed by storage engine, just check time in the file // the last chunk group may contain the same data with the logs, ignore such logs in seq // file diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java index 6756c88d329c..255b380ef0a1 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java @@ -127,7 +127,7 @@ protected void onFileEnd() { protected void onChunkGroup() throws IOException { FileLastTimeInfo fileNameLastTimePair = deviceEndTime.computeIfAbsent(currDeviceID, k -> new FileLastTimeInfo()); - if (!currDeviceID.equals(EMPTY_DEVICE_ID)) { + if (!currDeviceID.equals(EMPTY_DEVICE_ID) && !resource.definitelyNotContains(currDeviceID)) { long endTime = resource.getEndTime(currDeviceID); // record the end time of last device in current file if (endTime > fileNameLastTimePair.lastTime) { From 43fed136b6ca0b7682bc8c208216b055e2fbe962 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Thu, 19 Dec 2024 15:36:18 +0800 Subject: [PATCH 05/12] fix tests and use optional --- .../load/LoadTsFileTableSchemaCache.java | 12 +---- .../load/LoadTsFileTreeSchemaCache.java | 8 +--- .../plan/node/load/LoadSingleTsFileNode.java | 19 ++++---- .../storageengine/dataregion/DataRegion.java | 13 +++-- .../impl/FastCompactionPerformer.java | 4 +- .../utils/MultiTsFileDeviceIterator.java | 9 ++-- .../fast/SeriesCompactionExecutor.java | 19 ++++---- .../writer/AbstractCrossCompactionWriter.java | 6 ++- .../repair/RepairDataFileScanUtil.java | 47 ++++++++++++------- .../AbstractCompactionEstimator.java | 9 ++-- .../selector/impl/SettleSelectorImpl.java | 22 +++++---- .../utils/TsFileResourceCandidate.java | 12 ++++- .../dataregion/tsfile/TsFileResource.java | 32 +++++++++---- .../timeindex/ArrayDeviceTimeIndex.java | 20 ++++---- .../tsfile/timeindex/FileTimeIndex.java | 9 ++-- .../tsfile/timeindex/ITimeIndex.java | 5 +- .../TsFileDeviceStartEndTimeIterator.java | 6 ++- .../dataregion/utils/TsFileResourceUtils.java | 12 +++-- .../wal/recover/file/TsFilePlanRedoer.java | 9 ++-- .../file/UnsealedTsFileRecoverPerformer.java | 10 ++-- .../iotdb/db/tools/IoTDBDataDirViewer.java | 6 ++- .../iotdb/db/tools/TsFileResourcePrinter.java | 10 ++-- .../db/tools/utils/TsFileValidationScan.java | 13 ++--- .../TsFileOverlapValidationAndRepairTool.java | 8 ++-- .../iotdb/db/utils/ModificationUtils.java | 17 +++++++ ...actionWithFastPerformerValidationTest.java | 14 +++--- ...nWithReadPointPerformerValidationTest.java | 14 +++--- ...ertionCrossSpaceCompactionRecoverTest.java | 4 +- ...rtionCrossSpaceCompactionSelectorTest.java | 4 +- .../RewriteCompactionFileSelectorTest.java | 6 ++- .../utils/CompactionCheckerUtils.java | 5 +- .../memtable/TsFileProcessorTest.java | 3 +- 32 files changed, 241 insertions(+), 146 deletions(-) diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java index ac715923945d..40db65fbead4 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTableSchemaCache.java @@ -115,7 +115,7 @@ public void setDatabase(String database) { public void autoCreateAndVerify(IDeviceID device) { try { - if (isDeviceDeletedByMods(device)) { + if (ModificationUtils.isDeviceDeletedByMods(currentModifications, currentTimeIndex, device)) { return; } } catch (IllegalPathException e) { @@ -132,16 +132,6 @@ public void autoCreateAndVerify(IDeviceID device) { } } - private boolean isDeviceDeletedByMods(IDeviceID device) throws IllegalPathException { - return currentTimeIndex != null - && currentTimeIndex.checkDeviceIdExist(device) - && ModificationUtils.isAllDeletedByMods( - currentModifications, - device, - currentTimeIndex.getStartTime(device), - currentTimeIndex.getEndTime(device)); - } - private void addDevice(final IDeviceID device) { final String tableName = device.getTableName(); long memoryUsageSizeInBytes = 0; diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java index c6e215e67bd6..11bdf353f26b 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/load/LoadTsFileTreeSchemaCache.java @@ -188,13 +188,7 @@ public void setCurrentModificationsAndTimeIndex( } public boolean isDeviceDeletedByMods(IDeviceID device) throws IllegalPathException { - return currentTimeIndex != null - && currentTimeIndex.checkDeviceIdExist(device) - && ModificationUtils.isAllDeletedByMods( - currentModifications, - device, - currentTimeIndex.getStartTime(device), - currentTimeIndex.getEndTime(device)); + return ModificationUtils.isDeviceDeletedByMods(currentModifications, currentTimeIndex, device); } public boolean isTimeseriesDeletedByMods(IDeviceID device, TimeseriesMetadata timeseriesMetadata) diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java index c9d7567ad9ee..c8691868c826 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/node/load/LoadSingleTsFileNode.java @@ -85,21 +85,22 @@ public boolean isTsFileEmpty() { return resource.getDevices().isEmpty(); } + @SuppressWarnings("OptionalGetWithoutIsPresent") public boolean needDecodeTsFile( - Function>, List> partitionFetcher) - throws IOException { + Function>, List> + partitionFetcher) { List> slotList = new ArrayList<>(); resource .getDevices() .forEach( o -> { - if (!resource.definitelyNotContains(o)) { - slotList.add( - new Pair<>( - o, TimePartitionUtils.getTimePartitionSlot(resource.getStartTime(o)))); - slotList.add( - new Pair<>(o, TimePartitionUtils.getTimePartitionSlot(resource.getEndTime(o)))); - } + // iterating the index, must present + slotList.add( + new Pair<>( + o, TimePartitionUtils.getTimePartitionSlot(resource.getStartTime(o).get()))); + slotList.add( + new Pair<>( + o, TimePartitionUtils.getTimePartitionSlot(resource.getEndTime(o).get()))); }); if (slotList.isEmpty()) { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/DataRegion.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/DataRegion.java index 9cfe7e324b3d..4019cf732365 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/DataRegion.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/DataRegion.java @@ -666,7 +666,8 @@ protected void updateDeviceLastFlushTime(TsFileResource resource) { long timePartitionId = resource.getTimePartition(); Map endTimeMap = new HashMap<>(); for (IDeviceID deviceId : resource.getDevices()) { - long endTime = resource.getEndTime(deviceId); + @SuppressWarnings("OptionalGetWithoutIsPresent") // checked above + long endTime = resource.getEndTime(deviceId).get(); endTimeMap.put(deviceId, endTime); } if (config.isEnableSeparateData()) { @@ -682,7 +683,9 @@ protected void upgradeAndUpdateDeviceLastFlushTime( Map endTimeMap = new HashMap<>(); for (TsFileResource resource : resources) { for (IDeviceID deviceId : resource.getDevices()) { - long endTime = resource.getEndTime(deviceId); + // checked above + //noinspection OptionalGetWithoutIsPresent + long endTime = resource.getEndTime(deviceId).get(); endTimeMap.put(deviceId, endTime); } } @@ -2429,6 +2432,7 @@ public void insertSeparatorToWAL() { new ContinuousSameSearchIndexSeparatorNode())); } + @SuppressWarnings("OptionalGetWithoutIsPresent") private boolean canSkipDelete(TsFileResource tsFileResource, ModEntry deletion) { long fileStartTime = tsFileResource.getTimeIndex().getMinStartTime(); long fileEndTime = @@ -2446,10 +2450,11 @@ private boolean canSkipDelete(TsFileResource tsFileResource, ModEntry deletion) } for (IDeviceID device : tsFileResource.getDevices()) { - long startTime = tsFileResource.getTimeIndex().getStartTime(device); + // we are iterating the time index so the times are definitely present + long startTime = tsFileResource.getTimeIndex().getStartTime(device).get(); long endTime = tsFileResource.isClosed() - ? tsFileResource.getTimeIndex().getEndTime(device) + ? tsFileResource.getTimeIndex().getEndTime(device).get() : Long.MAX_VALUE; if (deletion.affects(device, startTime, endTime)) { return false; diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/performer/impl/FastCompactionPerformer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/performer/impl/FastCompactionPerformer.java index f855fd15f001..a1da2ff0eaee 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/performer/impl/FastCompactionPerformer.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/performer/impl/FastCompactionPerformer.java @@ -139,7 +139,9 @@ public void perform() throws Exception { long ttl = deviceIterator.getTTLForCurrentDevice(); sortedSourceFiles.removeIf( x -> x.definitelyNotContains(device) || !x.isDeviceAlive(device, ttl)); - sortedSourceFiles.sort(Comparator.comparingLong(x -> x.getStartTime(device))); + // checked above + //noinspection OptionalGetWithoutIsPresent + sortedSourceFiles.sort(Comparator.comparingLong(x -> x.getStartTime(device).get())); if (ttl != Long.MAX_VALUE) { ModEntry ttlDeletion = CompactionUtils.convertTtlToDeletion( diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java index 1e153551ead9..27c499800192 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/MultiTsFileDeviceIterator.java @@ -54,6 +54,7 @@ import java.util.LinkedList; import java.util.List; import java.util.Map; +import java.util.Optional; import java.util.TreeMap; import java.util.concurrent.ConcurrentHashMap; import java.util.stream.Collectors; @@ -425,8 +426,8 @@ private void applyModificationForAlignedChunkMetadataList( } IDeviceID device = currentDevice.getLeft(); ModEntry ttlDeletion = null; - if (!tsFileResource.definitelyNotContains(device) - && tsFileResource.getStartTime(device) < timeLowerBoundForCurrentDevice) { + Optional startTime = tsFileResource.getStartTime(device); + if (startTime.isPresent() && startTime.get() < timeLowerBoundForCurrentDevice) { ttlDeletion = CompactionUtils.convertTtlToDeletion(device, timeLowerBoundForCurrentDevice); } @@ -663,8 +664,8 @@ public String nextSeries() throws IllegalPathException { Map> chunkMetadataListMap = chunkMetadataCacheMap.get(reader); ModEntry ttlDeletion = null; - if (!resource.definitelyNotContains(device) - && resource.getStartTime(device) < timeLowerBoundForCurrentDevice) { + Optional startTime = resource.getStartTime(device); + if (startTime.isPresent() && startTime.get() < timeLowerBoundForCurrentDevice) { ttlDeletion = new TreeDeletionEntry( new MeasurementPath(device, IoTDBConstant.ONE_LEVEL_PATH_WILDCARD), diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java index d631a272e986..e9a3472fe115 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/fast/SeriesCompactionExecutor.java @@ -49,6 +49,7 @@ import java.util.Collections; import java.util.List; import java.util.Map; +import java.util.Optional; import java.util.PriorityQueue; public abstract class SeriesCompactionExecutor { @@ -349,17 +350,19 @@ private void checkAndCompactOverlapPage(PageElement nextPageElement, TimeValuePa * list is ordered according to the startTime of the current device in the file from small to * large, so that each file can be compacted in order. */ - protected List findOverlapFiles(FileElement file) { + protected List findOverlapFiles(FileElement fileToCheck) { List overlappedFiles = new ArrayList<>(); - long endTime = file.resource.getEndTime(deviceId); - for (FileElement fileElement : fileList) { - if (fileElement.resource.definitelyNotContains(deviceId)) { + Optional endTimeInCheckingFile = fileToCheck.resource.getEndTime(deviceId); + for (FileElement otherFile : fileList) { + if (!endTimeInCheckingFile.isPresent()) { continue; } - if (fileElement.resource.getStartTime(deviceId) <= endTime) { - if (!fileElement.isSelected) { - overlappedFiles.add(fileElement); - fileElement.isSelected = true; + Optional startTimeInOtherFile = otherFile.resource.getStartTime(deviceId); + if (startTimeInOtherFile.isPresent() + && startTimeInOtherFile.get() <= endTimeInCheckingFile.get()) { + if (!otherFile.isSelected) { + overlappedFiles.add(otherFile); + otherFile.isSelected = true; } } else { break; diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java index d0e3698bd736..efab75d42795 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java @@ -207,7 +207,11 @@ private void checkIsDeviceExistAndGetDeviceEndTime() throws IOException { while (fileIndex < seqTsFileResources.size()) { if (seqTsFileResources.get(fileIndex).getTimeIndexType() == 1) { // the timeIndexType of resource is deviceTimeIndex - currentDeviceEndTime[fileIndex] = seqTsFileResources.get(fileIndex).getEndTime(deviceId); + int finalFileIndex = fileIndex; + seqTsFileResources + .get(fileIndex) + .getEndTime(deviceId) + .ifPresent(endTime -> currentDeviceEndTime[finalFileIndex] = endTime); } else { long endTime = Long.MIN_VALUE; // Fast compaction get reader from cache map, while read point compaction get reader from diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java index 2d268761ecab..7d8fb91986d4 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java @@ -63,6 +63,7 @@ import java.util.HashSet; import java.util.List; import java.util.Map; +import java.util.Optional; import java.util.Set; import static org.apache.tsfile.read.reader.chunk.ChunkReader.decryptAndUncompressPageData; @@ -109,32 +110,44 @@ public void scanTsFile(boolean checkTsFileResource) { resource.isSeq() ? CompactionType.INNER_SEQ_COMPACTION : CompactionType.INNER_UNSEQ_COMPACTION)) { - TsFileDeviceIterator deviceIterator = reader.getAllDevicesIteratorWithIsAligned(); + TsFileDeviceIterator deviceInfileIterator = reader.getAllDevicesIteratorWithIsAligned(); Set deviceIdsInTimeIndex = checkTsFileResource ? new HashSet<>(timeIndex.getDevices()) : Collections.emptySet(); - while (deviceIterator.hasNext()) { - Pair deviceIsAlignedPair = deviceIterator.next(); - IDeviceID device = deviceIsAlignedPair.getLeft(); + while (deviceInfileIterator.hasNext()) { + Pair deviceIsAlignedPair = deviceInfileIterator.next(); + IDeviceID deviceInfile = deviceIsAlignedPair.getLeft(); if (checkTsFileResource) { - if (!deviceIdsInTimeIndex.contains(device)) { + if (!deviceIdsInTimeIndex.contains(deviceInfile)) { throw new CompactionStatisticsCheckFailedException( - device + " does not exist in the resource file"); + deviceInfile + " does not exist in the resource file"); } - deviceIdsInTimeIndex.remove(device); + deviceIdsInTimeIndex.remove(deviceInfile); } MetadataIndexNode metadataIndexNode = - deviceIterator.getFirstMeasurementNodeOfCurrentDevice(); + deviceInfileIterator.getFirstMeasurementNodeOfCurrentDevice(); + + Optional startTime = timeIndex.getStartTime(deviceInfile); + Optional endTime = timeIndex.getEndTime(deviceInfile); + if (!startTime.isPresent() || !endTime.isPresent()) { + continue; + } TimeRange deviceTimeRangeInResource = - checkTsFileResource - ? new TimeRange(timeIndex.getStartTime(device), timeIndex.getEndTime(device)) - : null; + checkTsFileResource ? new TimeRange(startTime.get(), endTime.get()) : null; boolean isAligned = deviceIsAlignedPair.getRight(); if (isAligned) { checkAlignedDeviceSeries( - reader, device, metadataIndexNode, deviceTimeRangeInResource, checkTsFileResource); + reader, + deviceInfile, + metadataIndexNode, + deviceTimeRangeInResource, + checkTsFileResource); } else { checkNonAlignedDeviceSeries( - reader, device, metadataIndexNode, deviceTimeRangeInResource, checkTsFileResource); + reader, + deviceInfile, + metadataIndexNode, + deviceTimeRangeInResource, + checkTsFileResource); } } if (!deviceIdsInTimeIndex.isEmpty()) { @@ -413,6 +426,7 @@ public boolean isBrokenFile() { return isBrokenFile; } + @SuppressWarnings("OptionalGetWithoutIsPresent") public static List checkTimePartitionHasOverlap( List resources, boolean printOverlappedDevices) { List overlapResources = new ArrayList<>(); @@ -433,8 +447,9 @@ public static List checkTimePartitionHasOverlap( boolean fileHasOverlap = false; // check overlap for (IDeviceID device : devices) { - long deviceStartTimeInCurrentFile = deviceTimeIndex.getStartTime(device); - if (deviceStartTimeInCurrentFile > deviceTimeIndex.getEndTime(device)) { + // we are iterating the time index so the times are definitely present + long deviceStartTimeInCurrentFile = deviceTimeIndex.getStartTime(device).get(); + if (deviceStartTimeInCurrentFile > deviceTimeIndex.getEndTime(device).get()) { continue; } if (!deviceEndTimeMap.containsKey(device)) { @@ -457,7 +472,7 @@ public static List checkTimePartitionHasOverlap( // update end time map if (!fileHasOverlap) { for (IDeviceID device : devices) { - deviceEndTimeMap.put(device, deviceTimeIndex.getEndTime(device)); + deviceEndTimeMap.put(device, deviceTimeIndex.getEndTime(device).get()); } } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java index fb9d825a5918..c4257fd715aa 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java @@ -117,6 +117,7 @@ private FileInfo getFileInfoFromCache(TsFileResource resource) throws IOExceptio } } + @SuppressWarnings("OptionalGetWithoutIsPresent") protected int calculatingMaxOverlapFileNumInSubCompactionTask(List resources) throws IOException { Set devices = new HashSet<>(); @@ -128,10 +129,11 @@ protected int calculatingMaxOverlapFileNumInSubCompactionTask(List resourcesContainsCurrentDevice = resourceDevices.stream() .filter(resource -> !resource.definitelyNotContains(device)) - .sorted(Comparator.comparingLong(resource -> resource.getStartTime(device))) + .sorted(Comparator.comparingLong(resource -> resource.getStartTime(device).get())) .collect(Collectors.toList()); if (resourcesContainsCurrentDevice.size() < maxOverlapFileNumInSubCompactionTask) { continue; @@ -143,8 +145,9 @@ protected int calculatingMaxOverlapFileNumInSubCompactionTask(List modifications = resource.getAllModEntries(); @@ -218,23 +219,28 @@ private FileDirtyInfo selectFileBaseOnDirtyData(TsFileResource resource) throws ttl = DataNodeTTLCache.getInstance().getTTLForTable(storageGroupName, tableName); } boolean hasSetTTL = ttl != Long.MAX_VALUE; + + long endTime = timeIndex.getEndTime(device).get(); boolean isDeleted = !timeIndex.isDeviceAlive(device, ttl) || isDeviceDeletedByMods( - modifications, - device, - timeIndex.getStartTime(device), - timeIndex.getEndTime(device)); - + modifications, device, timeIndex.getStartTime(device).get(), endTime); if (hasSetTTL) { if (!isDeleted) { // For devices with TTL set, all data must expire in order to meet the conditions for // being selected. return new FileDirtyInfo(DirtyStatus.NOT_SATISFIED); } - long outdatedTimeDiff = currentTime - timeIndex.getEndTime(device); - hasExpiredTooLong = - hasExpiredTooLong || outdatedTimeDiff > Math.min(config.getMaxExpiredTime(), 3 * ttl); + + if (currentTime > endTime) { + long outdatedTimeDiff = currentTime - endTime; + if (endTime < 0 && outdatedTimeDiff < currentTime) { + // overflow, like 100 - Long.MIN + outdatedTimeDiff = Long.MAX_VALUE; + } + hasExpiredTooLong = + hasExpiredTooLong || outdatedTimeDiff > Math.min(config.getMaxExpiredTime(), 3 * ttl); + } // else hasExpiredTooLong unchanged } if (isDeleted) { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/utils/TsFileResourceCandidate.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/utils/TsFileResourceCandidate.java index 7356f1aa6248..617b90b366fd 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/utils/TsFileResourceCandidate.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/utils/TsFileResourceCandidate.java @@ -92,20 +92,28 @@ private void prepareDeviceInfos() throws IOException { } ArrayDeviceTimeIndex timeIndex = CompactionUtils.buildDeviceTimeIndex(resource); for (IDeviceID deviceId : timeIndex.getDevices()) { + // iterating the index, must present + //noinspection OptionalGetWithoutIsPresent deviceInfoMap.put( deviceId, new DeviceInfo( - deviceId, timeIndex.getStartTime(deviceId), timeIndex.getEndTime(deviceId))); + deviceId, + timeIndex.getStartTime(deviceId).get(), + timeIndex.getEndTime(deviceId).get())); } } finally { resource.readUnlock(); } } else { for (IDeviceID deviceId : resource.getDevices()) { + // iterating the index, must present + //noinspection OptionalGetWithoutIsPresent deviceInfoMap.put( deviceId, new DeviceInfo( - deviceId, resource.getStartTime(deviceId), resource.getEndTime(deviceId))); + deviceId, + resource.getStartTime(deviceId).get(), + resource.getEndTime(deviceId).get())); } } hasDetailedDeviceInfo = true; diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java index e1b87ec0e36b..11778dda325b 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResource.java @@ -81,6 +81,7 @@ import java.util.Map; import java.util.NoSuchElementException; import java.util.Objects; +import java.util.Optional; import java.util.Set; import java.util.concurrent.CompletableFuture; import java.util.concurrent.ExecutionException; @@ -574,9 +575,9 @@ public long getTsFileSize() { } } - public long getStartTime(IDeviceID deviceId) { + public Optional getStartTime(IDeviceID deviceId) { try { - return deviceId == null ? getFileStartTime() : timeIndex.getStartTime(deviceId); + return deviceId == null ? Optional.of(getFileStartTime()) : timeIndex.getStartTime(deviceId); } catch (Exception e) { LOGGER.error( "meet error when getStartTime of {} in file {}", deviceId, file.getAbsolutePath(), e); @@ -588,9 +589,9 @@ public long getStartTime(IDeviceID deviceId) { } /** open file's end time is Long.MIN_VALUE */ - public long getEndTime(IDeviceID deviceId) { + public Optional getEndTime(IDeviceID deviceId) { try { - return deviceId == null ? getFileEndTime() : timeIndex.getEndTime(deviceId); + return deviceId == null ? Optional.of(getFileEndTime()) : timeIndex.getEndTime(deviceId); } catch (Exception e) { LOGGER.error( "meet error when getEndTime of {} in file {}", deviceId, file.getAbsolutePath(), e); @@ -604,7 +605,9 @@ public long getEndTime(IDeviceID deviceId) { // cannot use FileTimeIndex public long getOrderTimeForSeq(IDeviceID deviceId, boolean ascending) { if (timeIndex instanceof ArrayDeviceTimeIndex && !definitelyNotContains(deviceId)) { - return ascending ? getStartTime(deviceId) : getEndTime(deviceId); + // checked above + //noinspection OptionalGetWithoutIsPresent + return ascending ? getStartTime(deviceId).get() : getEndTime(deviceId).get(); } else { return ascending ? Long.MIN_VALUE : Long.MAX_VALUE; } @@ -613,7 +616,9 @@ public long getOrderTimeForSeq(IDeviceID deviceId, boolean ascending) { // can use FileTimeIndex public long getOrderTimeForUnseq(IDeviceID deviceId, boolean ascending) { if (!definitelyNotContains(deviceId)) { - return ascending ? getStartTime(deviceId) : getEndTime(deviceId); + // checked above + //noinspection OptionalGetWithoutIsPresent + return ascending ? getStartTime(deviceId).get() : getEndTime(deviceId).get(); } else { return ascending ? Long.MIN_VALUE : Long.MAX_VALUE; } @@ -948,6 +953,7 @@ public boolean isDeviceIdExist(IDeviceID deviceId) { /** * @return true if the device is contained in the TsFile */ + @SuppressWarnings("OptionalGetWithoutIsPresent") public boolean isSatisfied(IDeviceID deviceId, Filter timeFilter, boolean isSeq, boolean debug) { if (deviceId != null && definitelyNotContains(deviceId)) { if (debug) { @@ -957,8 +963,9 @@ public boolean isSatisfied(IDeviceID deviceId, Filter timeFilter, boolean isSeq, return false; } - long startTime = getStartTime(deviceId); - long endTime = isClosed() || !isSeq ? getEndTime(deviceId) : Long.MAX_VALUE; + // check above + long startTime = getStartTime(deviceId).get(); + long endTime = isClosed() || !isSeq ? getEndTime(deviceId).get() : Long.MAX_VALUE; if (startTime > endTime) { // startTime > endTime indicates that there is something wrong with this TsFile. Return false // directly, or it may lead to infinite loop in GroupByMonthFilter#getTimePointPosition. @@ -1299,8 +1306,13 @@ private void generatePathToTimeSeriesMetadataMap() throws IOException { public void deleteRemovedDeviceAndUpdateEndTime(Map lastTimeForEachDevice) { ITimeIndex newTimeIndex = CONFIG.getTimeIndexLevel().getTimeIndex(); for (Map.Entry entry : lastTimeForEachDevice.entrySet()) { - newTimeIndex.updateStartTime(entry.getKey(), timeIndex.getStartTime(entry.getKey())); - newTimeIndex.updateEndTime(entry.getKey(), entry.getValue()); + timeIndex + .getStartTime(entry.getKey()) + .ifPresent( + startTime -> { + newTimeIndex.updateStartTime(entry.getKey(), startTime); + newTimeIndex.updateEndTime(entry.getKey(), entry.getValue()); + }); } timeIndex = newTimeIndex; } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java index 8b99caca8c54..663595808c61 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ArrayDeviceTimeIndex.java @@ -43,7 +43,7 @@ import java.util.HashSet; import java.util.Map; import java.util.Map.Entry; -import java.util.NoSuchElementException; +import java.util.Optional; import java.util.Set; import java.util.concurrent.ConcurrentHashMap; @@ -320,7 +320,8 @@ private long getTimePartitionWithCheck() throws PartitionViolationException { @Override public void updateStartTime(IDeviceID deviceId, long time) { int index = getDeviceIndex(deviceId); - long startTime = getStartTime(deviceId); + @SuppressWarnings("OptionalGetWithoutIsPresent") // must present after getDeviceIndex + long startTime = getStartTime(deviceId).get(); if (time < startTime) { startTimes[index] = time; } @@ -330,7 +331,8 @@ public void updateStartTime(IDeviceID deviceId, long time) { @Override public void updateEndTime(IDeviceID deviceId, long time) { int index = getDeviceIndex(deviceId); - long endTime = getEndTime(deviceId); + @SuppressWarnings("OptionalGetWithoutIsPresent") // must present after getDeviceIndex + long endTime = getEndTime(deviceId).get(); if (time > endTime) { endTimes[index] = time; } @@ -352,19 +354,19 @@ public void putEndTime(IDeviceID deviceId, long time) { } @Override - public long getStartTime(IDeviceID deviceId) { + public Optional getStartTime(IDeviceID deviceId) { if (!deviceToIndex.containsKey(deviceId)) { - throw new NoSuchElementException(deviceId.toString()); + return Optional.empty(); } - return startTimes[deviceToIndex.get(deviceId)]; + return Optional.of(startTimes[deviceToIndex.get(deviceId)]); } @Override - public long getEndTime(IDeviceID deviceId) { + public Optional getEndTime(IDeviceID deviceId) { if (!deviceToIndex.containsKey(deviceId)) { - throw new NoSuchElementException(deviceId.toString()); + return Optional.empty(); } - return endTimes[deviceToIndex.get(deviceId)]; + return Optional.of(endTimes[deviceToIndex.get(deviceId)]); } @Override diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/FileTimeIndex.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/FileTimeIndex.java index c9820d809f1e..3e1b1062d649 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/FileTimeIndex.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/FileTimeIndex.java @@ -40,6 +40,7 @@ import java.nio.ByteBuffer; import java.nio.file.NoSuchFileException; import java.util.Collections; +import java.util.Optional; import java.util.Set; public class FileTimeIndex implements ITimeIndex { @@ -190,8 +191,8 @@ public void putEndTime(IDeviceID deviceId, long time) { } @Override - public long getStartTime(IDeviceID deviceId) { - return startTime; + public Optional getStartTime(IDeviceID deviceId) { + return Optional.of(startTime); } @Override @@ -200,8 +201,8 @@ public long getMinStartTime() { } @Override - public long getEndTime(IDeviceID deviceId) { - return endTime; + public Optional getEndTime(IDeviceID deviceId) { + return Optional.of(endTime); } @Override diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ITimeIndex.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ITimeIndex.java index a966d3036440..4711fcec0ebd 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ITimeIndex.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/timeindex/ITimeIndex.java @@ -31,6 +31,7 @@ import java.io.InputStream; import java.io.OutputStream; import java.nio.ByteBuffer; +import java.util.Optional; import java.util.Set; public interface ITimeIndex { @@ -150,7 +151,7 @@ public interface ITimeIndex { * @param deviceId device name * @return start time */ - long getStartTime(IDeviceID deviceId); + Optional getStartTime(IDeviceID deviceId); /** * get end time of device @@ -158,7 +159,7 @@ public interface ITimeIndex { * @param deviceId device name * @return end time */ - long getEndTime(IDeviceID deviceId); + Optional getEndTime(IDeviceID deviceId); /** * check whether deviceId exists in TsFile diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileDeviceStartEndTimeIterator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileDeviceStartEndTimeIterator.java index fd0fff8cba9d..557d0764c1fc 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileDeviceStartEndTimeIterator.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileDeviceStartEndTimeIterator.java @@ -46,7 +46,11 @@ public boolean hasNext() { public DeviceStartEndTime next() { IDeviceID deviceID = currentDevice.next(); + // iterating the index, must present + //noinspection OptionalGetWithoutIsPresent return new DeviceStartEndTime( - deviceID, deviceTimeIndex.getStartTime(deviceID), deviceTimeIndex.getEndTime(deviceID)); + deviceID, + deviceTimeIndex.getStartTime(deviceID).get(), + deviceTimeIndex.getEndTime(deviceID).get()); } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java index 679d5119b9ec..6ff2b605cf45 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/utils/TsFileResourceUtils.java @@ -70,6 +70,7 @@ private TsFileResourceUtils() { // util class } + @SuppressWarnings("OptionalGetWithoutIsPresent") public static boolean validateTsFileResourceCorrectness(TsFileResource resource) { if (resource.isDeleted()) { return true; @@ -92,8 +93,9 @@ public static boolean validateTsFileResourceCorrectness(TsFileResource resource) return false; } for (IDeviceID device : devices) { - long startTime = timeIndex.getStartTime(device); - long endTime = timeIndex.getEndTime(device); + // iterating the index, must present + long startTime = timeIndex.getStartTime(device).get(); + long endTime = timeIndex.getEndTime(device).get(); if (startTime > endTime) { logger.error( "{} {} the start time of {} is greater than end time", @@ -357,6 +359,7 @@ public static Map getChunkMetadata(TsFileSequenceReader re return offset2ChunkMetadata; } + @SuppressWarnings("OptionalGetWithoutIsPresent") public static boolean validateTsFileResourcesHasNoOverlap(List resources) { // deviceID -> Map> lastEndTimeMap = new HashMap<>(); @@ -378,8 +381,9 @@ public static boolean validateTsFileResourcesHasNoOverlap(List r } Set devices = timeIndex.getDevices(); for (IDeviceID device : devices) { - long currentStartTime = timeIndex.getStartTime(device); - long currentEndTime = timeIndex.getEndTime(device); + // iterating the index, must present + long currentStartTime = timeIndex.getStartTime(device).get(); + long currentEndTime = timeIndex.getEndTime(device).get(); Pair lastDeviceInfo = lastEndTimeMap.computeIfAbsent(device, x -> new Pair<>(null, Long.MIN_VALUE)); long lastEndTime = lastDeviceInfo.right; diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java index 159382a84968..a90fb80511fa 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java @@ -42,6 +42,7 @@ import java.io.IOException; import java.util.ArrayList; import java.util.List; +import java.util.Optional; /** * This class helps redo wal logs into a TsFile. Notice: You should update time map in {@link @@ -88,14 +89,14 @@ void redoInsert(InsertNode node) throws WriteProcessException { if (tsFileResource != null && !tsFileResource.definitelyNotContains(node.getDeviceID())) { // orders of insert node is guaranteed by storage engine, just check time in the file // the last chunk group may contain the same data with the logs, ignore such logs in seq file - long lastEndTime = tsFileResource.getEndTime(node.getDeviceID()); + Optional lastEndTime = tsFileResource.getEndTime(node.getDeviceID()); long minTimeInNode; if (node instanceof InsertRowNode) { minTimeInNode = ((InsertRowNode) node).getTime(); } else { minTimeInNode = ((InsertTabletNode) node).getTimes()[0]; } - if (lastEndTime != Long.MIN_VALUE && lastEndTime >= minTimeInNode) { + if (lastEndTime.isPresent() && lastEndTime.get() >= minTimeInNode) { return; } } @@ -131,10 +132,10 @@ void redoInsertRows(InsertRowsNode insertRowsNode) { // orders of insert node is guaranteed by storage engine, just check time in the file // the last chunk group may contain the same data with the logs, ignore such logs in seq // file - long lastEndTime = tsFileResource.getEndTime(node.getDeviceID()); + Optional lastEndTime = tsFileResource.getEndTime(node.getDeviceID()); long minTimeInNode; minTimeInNode = node.getTime(); - if (lastEndTime != Long.MIN_VALUE && lastEndTime >= minTimeInNode) { + if (lastEndTime.isPresent() && lastEndTime.get() >= minTimeInNode) { continue; } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformer.java index a0901f8772cc..c955f67c79c0 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformer.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformer.java @@ -161,6 +161,7 @@ private void constructResourceFromTsFile() { } /** Redo log. */ + @SuppressWarnings("OptionalGetWithoutIsPresent") public void redoLog(WALEntry walEntry) { // skip redo wal log when this TsFile is not crashed if (!hasCrashed()) { @@ -177,19 +178,20 @@ public void redoLog(WALEntry walEntry) { if (tsFileResource != null) { // delete data already flushed in the MemTable to avoid duplicates for (IDeviceID device : tsFileResource.getDevices()) { + // iterating the index, must present if (device.isTableModel()) { memTable.delete( new TableDeletionEntry( new DeletionPredicate(device.getTableName(), new FullExactMatch(device)), new TimeRange( - tsFileResource.getStartTime(device), - tsFileResource.getEndTime(device)))); + tsFileResource.getStartTime(device).get(), + tsFileResource.getEndTime(device).get()))); } else { memTable.delete( new TreeDeletionEntry( new MeasurementPath(device, "*"), - tsFileResource.getStartTime(device), - tsFileResource.getEndTime(device))); + tsFileResource.getStartTime(device).get(), + tsFileResource.getEndTime(device).get())); } } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/IoTDBDataDirViewer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/IoTDBDataDirViewer.java index 6713c51e05ec..4c174be196ec 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/IoTDBDataDirViewer.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/IoTDBDataDirViewer.java @@ -164,15 +164,17 @@ private static void printResource(String filename, PrintWriter pw) throws IOExce // sort device strings SortedSet keys = new TreeSet<>(resource.getDevices()); for (IDeviceID device : keys) { + // iterating the index, must present + //noinspection OptionalGetWithoutIsPresent printlnBoth( pw, String.format( "| | | | | |--device %s, start time %d (%s), end time %d (%s)", device, resource.getStartTime(device), - DateTimeUtils.convertLongToDate(resource.getStartTime(device)), + DateTimeUtils.convertLongToDate(resource.getStartTime(device).get()), resource.getEndTime(device), - DateTimeUtils.convertLongToDate(resource.getEndTime(device)))); + DateTimeUtils.convertLongToDate(resource.getEndTime(device).get()))); } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/TsFileResourcePrinter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/TsFileResourcePrinter.java index d3e5597ee3e5..7fe4ca1d01e4 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/TsFileResourcePrinter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/TsFileResourcePrinter.java @@ -77,13 +77,15 @@ public static void printResource(String filename) throws IOException { resource.getMinPlanIndex(), resource.getMaxPlanIndex()); for (IDeviceID device : resource.getDevices()) { + // iterating the index, must present + //noinspection OptionalGetWithoutIsPresent System.out.printf( "device %s, start time %d (%s), end time %d (%s)%n", device, - resource.getStartTime(device), - DateTimeUtils.convertLongToDate(resource.getStartTime(device)), - resource.getEndTime(device), - DateTimeUtils.convertLongToDate(resource.getEndTime(device))); + resource.getStartTime(device).get(), + DateTimeUtils.convertLongToDate(resource.getStartTime(device).get()), + resource.getEndTime(device).get(), + DateTimeUtils.convertLongToDate(resource.getEndTime(device).get())); } System.out.println(); } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java index 255b380ef0a1..80cde572d8b7 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/utils/TsFileValidationScan.java @@ -42,6 +42,7 @@ import java.util.HashMap; import java.util.List; import java.util.Map; +import java.util.Optional; @SuppressWarnings("java:S2175") public class TsFileValidationScan extends TsFileSequenceScan { @@ -127,12 +128,12 @@ protected void onFileEnd() { protected void onChunkGroup() throws IOException { FileLastTimeInfo fileNameLastTimePair = deviceEndTime.computeIfAbsent(currDeviceID, k -> new FileLastTimeInfo()); - if (!currDeviceID.equals(EMPTY_DEVICE_ID) && !resource.definitelyNotContains(currDeviceID)) { - long endTime = resource.getEndTime(currDeviceID); + if (!currDeviceID.equals(EMPTY_DEVICE_ID)) { + Optional endTime = resource.getEndTime(currDeviceID); // record the end time of last device in current file - if (endTime > fileNameLastTimePair.lastTime) { + if (endTime.isPresent() && endTime.get() > fileNameLastTimePair.lastTime) { fileNameLastTimePair.lastFileName = file.getName(); - fileNameLastTimePair.endTimeInLastFile = endTime; + fileNameLastTimePair.endTimeInLastFile = endTime.get(); } } @@ -140,8 +141,8 @@ protected void onChunkGroup() throws IOException { fileNameLastTimePair = deviceEndTime.computeIfAbsent(currDeviceID, k -> new FileLastTimeInfo()); if (!Boolean.TRUE.equals(hasCheckedDeviceOverlap.getOrDefault(currDeviceID, false)) - && !resource.definitelyNotContains(currDeviceID) - && resource.getStartTime(currDeviceID) <= fileNameLastTimePair.endTimeInLastFile) { + && resource.getStartTime(currDeviceID).isPresent() + && resource.getStartTime(currDeviceID).get() <= fileNameLastTimePair.endTimeInLastFile) { // device overlap, find bad file recordDeviceOverlap(fileNameLastTimePair.lastFileName); } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/validate/TsFileOverlapValidationAndRepairTool.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/validate/TsFileOverlapValidationAndRepairTool.java index a383b99b462d..bf46e95e07b6 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/validate/TsFileOverlapValidationAndRepairTool.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/tools/validate/TsFileOverlapValidationAndRepairTool.java @@ -199,6 +199,7 @@ private static String calculateTimePartitionKey( return storageGroup + "-" + dataRegion + "-" + timePartition; } + @SuppressWarnings("OptionalGetWithoutIsPresent") public static int checkTimePartitionHasOverlap(List resources) { int overlapTsFileNum = 0; Map deviceEndTimeMap = new HashMap<>(); @@ -208,8 +209,9 @@ public static int checkTimePartitionHasOverlap(List resources) { boolean fileHasOverlap = false; // check overlap for (IDeviceID device : devices) { - long deviceStartTimeInCurrentFile = resource.getStartTime(device); - if (deviceStartTimeInCurrentFile > resource.getEndTime(device)) { + // iterating the index, must present + long deviceStartTimeInCurrentFile = resource.getStartTime(device).get(); + if (deviceStartTimeInCurrentFile > resource.getEndTime(device).get()) { continue; } if (!deviceEndTimeMap.containsKey(device)) { @@ -234,7 +236,7 @@ public static int checkTimePartitionHasOverlap(List resources) { // update end time map if (!fileHasOverlap) { for (IDeviceID device : devices) { - deviceEndTimeMap.put(device, resource.getEndTime(device)); + deviceEndTimeMap.put(device, resource.getEndTime(device).get()); deviceLastExistTsFileMap.put(device, resource); } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/ModificationUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/ModificationUtils.java index 599662076480..1267dcad61a9 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/ModificationUtils.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/utils/ModificationUtils.java @@ -19,11 +19,13 @@ package org.apache.iotdb.db.utils; +import org.apache.iotdb.commons.exception.IllegalPathException; import org.apache.iotdb.db.storageengine.dataregion.compaction.selector.impl.SettleSelectorImpl; import org.apache.iotdb.db.storageengine.dataregion.memtable.IMemTable; import org.apache.iotdb.db.storageengine.dataregion.modification.ModEntry; import org.apache.iotdb.db.storageengine.dataregion.modification.TableDeletionEntry; import org.apache.iotdb.db.storageengine.dataregion.modification.TreeDeletionEntry; +import org.apache.iotdb.db.storageengine.dataregion.tsfile.timeindex.ITimeIndex; import org.apache.tsfile.file.metadata.AlignedChunkMetadata; import org.apache.tsfile.file.metadata.IChunkMetadata; @@ -37,6 +39,7 @@ import java.util.Comparator; import java.util.List; import java.util.Objects; +import java.util.Optional; public class ModificationUtils { @@ -398,4 +401,18 @@ public static List sortAndMerge(List modifications) { } return result; } + + public static boolean isDeviceDeletedByMods( + Collection currentModifications, ITimeIndex currentTimeIndex, IDeviceID device) + throws IllegalPathException { + if (currentTimeIndex == null) { + return false; + } + Optional startTime = currentTimeIndex.getStartTime(device); + Optional endTime = currentTimeIndex.getEndTime(device); + if (startTime.isPresent() && endTime.isPresent()) { + return isAllDeletedByMods(currentModifications, device, startTime.get(), endTime.get()); + } + return false; + } } diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithFastPerformerValidationTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithFastPerformerValidationTest.java index b076755229ed..644ddf77e1f2 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithFastPerformerValidationTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithFastPerformerValidationTest.java @@ -76,6 +76,7 @@ import static org.apache.iotdb.commons.conf.IoTDBConstant.PATH_SEPARATOR; import static org.apache.tsfile.utils.TsFileGeneratorUtils.alignDeviceOffset; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class CrossSpaceCompactionWithFastPerformerValidationTest extends AbstractCompactionTest { TsFileManager tsFileManager = new TsFileManager(COMPACTION_TEST_SG, "0", STORAGE_GROUP_DIR.getPath()); @@ -1849,7 +1850,7 @@ public void testWithUnclosedSeqFile() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -1902,7 +1903,7 @@ public void testWithUnclosedSeqFileAndNewSensorInUnseqFile() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -1948,7 +1949,7 @@ public void testUnseqFileOverlapWithUnclosedSeqFile() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -2000,7 +2001,7 @@ public void testUnseqFileOverlapWithUnclosedSeqFile2() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -2053,8 +2054,9 @@ public void testWithUnclosedUnSeqFile() unclosedUnSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastUnSeqResource = unseqResources.get(1); for (IDeviceID deviceID : lastUnSeqResource.getDevices()) { - unclosedUnSeqResource.updateStartTime(deviceID, lastUnSeqResource.getStartTime(deviceID)); - unclosedUnSeqResource.updateEndTime(deviceID, lastUnSeqResource.getEndTime(deviceID)); + unclosedUnSeqResource.updateStartTime( + deviceID, lastUnSeqResource.getStartTime(deviceID).get()); + unclosedUnSeqResource.updateEndTime(deviceID, lastUnSeqResource.getEndTime(deviceID).get()); } unseqResources.remove(1); unseqResources.add(1, unclosedUnSeqResource); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithReadPointPerformerValidationTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithReadPointPerformerValidationTest.java index 77d5a66c301d..b05ab1e633e3 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithReadPointPerformerValidationTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/CrossSpaceCompactionWithReadPointPerformerValidationTest.java @@ -74,6 +74,7 @@ import static org.apache.iotdb.commons.conf.IoTDBConstant.PATH_SEPARATOR; import static org.apache.tsfile.utils.TsFileGeneratorUtils.alignDeviceOffset; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class CrossSpaceCompactionWithReadPointPerformerValidationTest extends AbstractCompactionTest { // TsFileManager tsFileManager = @@ -1848,7 +1849,7 @@ public void testWithUnclosedSeqFile() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -1901,7 +1902,7 @@ public void testWithUnclosedSeqFileAndNewSensorInUnseqFile() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -1947,7 +1948,7 @@ public void testUnseqFileOverlapWithUnclosedSeqFile() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -1999,7 +2000,7 @@ public void testUnseqFileOverlapWithUnclosedSeqFile2() unclosedSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastSeqResource = seqResources.get(4); for (IDeviceID deviceID : lastSeqResource.getDevices()) { - unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID)); + unclosedSeqResource.updateStartTime(deviceID, lastSeqResource.getStartTime(deviceID).get()); } seqResources.remove(4); seqResources.add(4, unclosedSeqResource); @@ -2052,8 +2053,9 @@ public void testWithUnclosedUnSeqFile() unclosedUnSeqResource.setStatusForTest(TsFileResourceStatus.UNCLOSED); TsFileResource lastUnSeqResource = unseqResources.get(1); for (IDeviceID deviceID : lastUnSeqResource.getDevices()) { - unclosedUnSeqResource.updateStartTime(deviceID, lastUnSeqResource.getStartTime(deviceID)); - unclosedUnSeqResource.updateEndTime(deviceID, lastUnSeqResource.getEndTime(deviceID)); + unclosedUnSeqResource.updateStartTime( + deviceID, lastUnSeqResource.getStartTime(deviceID).get()); + unclosedUnSeqResource.updateEndTime(deviceID, lastUnSeqResource.getEndTime(deviceID).get()); } unseqResources.remove(1); unseqResources.add(1, unclosedUnSeqResource); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionRecoverTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionRecoverTest.java index b89aef630bf1..75ce7382a4da 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionRecoverTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionRecoverTest.java @@ -62,6 +62,7 @@ import static org.apache.iotdb.db.storageengine.dataregion.compaction.utils.TsFileGeneratorUtils.writeNonAlignedChunk; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class InsertionCrossSpaceCompactionRecoverTest extends AbstractCompactionTest { @Before public void setUp() @@ -450,7 +451,8 @@ private void createTsFileByResource(TsFileResource resource) throws IOException new MeasurementSchema("s1", TSDataType.INT64, TSEncoding.PLAIN, CompressionType.SNAPPY); ChunkWriterImpl iChunkWriter = new ChunkWriterImpl(schema); List pages = new ArrayList<>(); - pages.add(new TimeRange(resource.getStartTime(device), resource.getEndTime(device))); + pages.add( + new TimeRange(resource.getStartTime(device).get(), resource.getEndTime(device).get())); writeNonAlignedChunk(iChunkWriter, tsFileIOWriter, pages, resource.isSeq()); tsFileIOWriter.endChunkGroup(); } diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionSelectorTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionSelectorTest.java index 6a86b3955f93..0852513388fe 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionSelectorTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/InsertionCrossSpaceCompactionSelectorTest.java @@ -57,6 +57,7 @@ import static org.apache.iotdb.db.storageengine.dataregion.compaction.utils.TsFileGeneratorUtils.writeNonAlignedChunk; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class InsertionCrossSpaceCompactionSelectorTest extends AbstractCompactionTest { @Before @@ -2029,7 +2030,8 @@ private void createTsFileByResource(TsFileResource resource) throws IOException new MeasurementSchema("s1", TSDataType.INT64, TSEncoding.PLAIN, CompressionType.SNAPPY); ChunkWriterImpl iChunkWriter = new ChunkWriterImpl(schema); List pages = new ArrayList<>(); - pages.add(new TimeRange(resource.getStartTime(device), resource.getEndTime(device))); + pages.add( + new TimeRange(resource.getStartTime(device).get(), resource.getEndTime(device).get())); writeNonAlignedChunk(iChunkWriter, tsFileIOWriter, pages, resource.isSeq()); tsFileIOWriter.endChunkGroup(); } diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/RewriteCompactionFileSelectorTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/RewriteCompactionFileSelectorTest.java index e67ba538da93..3afaca0cf1b3 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/RewriteCompactionFileSelectorTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/cross/RewriteCompactionFileSelectorTest.java @@ -178,7 +178,8 @@ public void testFileOpenSelection() ITimeIndex newTimeIndex = IoTDBDescriptor.getInstance().getConfig().getTimeIndexLevel().getTimeIndex(); for (IDeviceID device : devices) { - newTimeIndex.updateStartTime(device, timeIndex.getStartTime(device)); + //noinspection OptionalGetWithoutIsPresent + newTimeIndex.updateStartTime(device, timeIndex.getStartTime(device).get()); } secondTsFileResource.setTimeIndex(newTimeIndex); @@ -229,7 +230,8 @@ public void testFileOpenSelectionFromCompaction() ITimeIndex newTimeIndex = IoTDBDescriptor.getInstance().getConfig().getTimeIndexLevel().getTimeIndex(); for (IDeviceID device : devices) { - newTimeIndex.updateStartTime(device, timeIndex.getStartTime(device)); + //noinspection OptionalGetWithoutIsPresent + newTimeIndex.updateStartTime(device, timeIndex.getStartTime(device).get()); } secondTsFileResource.setTimeIndex(newTimeIndex); List newUnseqResources = new ArrayList<>(); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java index 13d3075be942..3e10cd0a978b 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/utils/CompactionCheckerUtils.java @@ -88,6 +88,7 @@ import static org.junit.Assert.assertNull; import static org.junit.Assert.fail; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class CompactionCheckerUtils { public static void putOnePageChunks( @@ -356,14 +357,14 @@ private static void checkTsFileResource( if (mergedFile.definitelyNotContains(device)) { continue; } - startTime = Math.min(startTime, mergedFile.getStartTime(device)); + startTime = Math.min(startTime, mergedFile.getStartTime(device).get()); } long endTime = Long.MIN_VALUE; for (TsFileResource mergedFile : mergedFiles) { if (mergedFile.definitelyNotContains(device)) { continue; } - endTime = Math.max(endTime, mergedFile.getEndTime(device)); + endTime = Math.max(endTime, mergedFile.getEndTime(device).get()); } assertEquals(statistics[0], startTime); assertEquals(statistics[1], endTime); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/memtable/TsFileProcessorTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/memtable/TsFileProcessorTest.java index 0393067f109f..6c41cb4ecb05 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/memtable/TsFileProcessorTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/memtable/TsFileProcessorTest.java @@ -70,6 +70,7 @@ import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertFalse; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class TsFileProcessorTest { private TsFileProcessor processor; @@ -866,7 +867,7 @@ private void closeTsFileProcessor(TsFileProcessor unsealedTsFileProcessor) TsFileResource resource = unsealedTsFileProcessor.getTsFileResource(); synchronized (resource) { for (IDeviceID deviceId : resource.getDevices()) { - resource.updateEndTime(deviceId, resource.getStartTime(deviceId)); + resource.updateEndTime(deviceId, resource.getStartTime(deviceId).get()); } try { resource.close(); From d083a7156a3541d5e1a6a7abd6e1f91968f9a5d6 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Thu, 19 Dec 2024 17:29:44 +0800 Subject: [PATCH 06/12] rename & self review --- .../it/session/IoTDBSessionRelationalIT.java | 126 ++++++++++++++++ .../session/it/IoTDBSessionSimpleIT.java | 58 +++++++- .../repair/RepairDataFileScanUtil.java | 22 +-- .../selector/impl/SettleSelectorImpl.java | 4 +- .../wal/recover/file/TsFilePlanRedoer.java | 4 +- .../load/splitter/AlignedChunkData.java | 4 +- .../BatchedAlignedValueChunkData.java | 4 +- .../CompactionDataTypeNotMatchTest.java | 19 ++- ...tCompactionPerformerWithEmptyPageTest.java | 5 +- .../FastInnerCompactionPerformerTest.java | 134 ++++++++++-------- .../dataregion/tsfile/TsFileResourceTest.java | 12 +- .../wal/recover/WALRecoverManagerTest.java | 21 +-- .../SealedTsFileRecoverPerformerTest.java | 33 ++--- .../UnsealedTsFileRecoverPerformerTest.java | 21 +-- .../resources/conf/iotdb-system.properties | 16 --- .../conf/iotdb-system.properties.template | 2 +- 16 files changed, 340 insertions(+), 145 deletions(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java b/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java index a57028dddbf6..fc831e576d62 100644 --- a/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java @@ -29,12 +29,16 @@ import org.apache.iotdb.rpc.StatementExecutionException; import org.apache.tsfile.enums.TSDataType; +import org.apache.tsfile.exception.write.WriteProcessException; +import org.apache.tsfile.file.metadata.TableSchema; import org.apache.tsfile.read.common.RowRecord; import org.apache.tsfile.utils.Binary; import org.apache.tsfile.write.record.Tablet; import org.apache.tsfile.write.record.Tablet.ColumnCategory; import org.apache.tsfile.write.schema.IMeasurementSchema; import org.apache.tsfile.write.schema.MeasurementSchema; +import org.apache.tsfile.write.v4.ITsFileWriter; +import org.apache.tsfile.write.v4.TsFileWriterBuilder; import org.junit.After; import org.junit.AfterClass; import org.junit.Before; @@ -43,6 +47,8 @@ import org.junit.experimental.categories.Category; import org.junit.runner.RunWith; +import java.io.File; +import java.io.IOException; import java.nio.charset.StandardCharsets; import java.time.LocalDate; import java.util.ArrayList; @@ -1464,4 +1470,124 @@ public void insertRelationalRowWithAutoCastTest() session.executeNonQueryStatement("SET CONFIGURATION \"enable_partial_insert\"=\"true\""); } } + + @Test + @Category({LocalStandaloneIT.class, ClusterIT.class}) + public void insertMinMaxTimeTest() throws IoTDBConnectionException, StatementExecutionException { + try { + try (ITableSession session = EnvFactory.getEnv().getTableSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION timestamp_precision_check_enabled='false'"); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + session.executeNonQueryStatement("USE db1"); + session.executeNonQueryStatement("CREATE TABLE test_insert_min_max (id1 ID, s1 INT32)"); + + session.executeNonQueryStatement( + String.format( + "INSERT INTO test_insert_min_max(time, id1, s1) VALUES (%d, 'd1', 1)", + Long.MIN_VALUE)); + session.executeNonQueryStatement( + String.format( + "INSERT INTO test_insert_min_max(time, id1, s1) VALUES (%d, 'd1', 1)", + Long.MAX_VALUE)); + + SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM test_insert_min_max"); + RowRecord record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getFields().get(0).getLongV()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getFields().get(0).getLongV()); + assertFalse(dataSet.hasNext()); + + session.executeNonQueryStatement("FLUSH"); + dataSet = session.executeQueryStatement("SELECT * FROM test_insert_min_max"); + record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getFields().get(0).getLongV()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getFields().get(0).getLongV()); + assertFalse(dataSet.hasNext()); + } + } finally { + try (ITableSession session = EnvFactory.getEnv().getTableSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION timestamp_precision_check_enabled='true'"); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + } + } + } + + @Test + @Category({LocalStandaloneIT.class, ClusterIT.class}) + public void loadMinMaxTimeAlignedTest() + throws IoTDBConnectionException, + StatementExecutionException, + IOException, + WriteProcessException { + File file = new File("target", "test.tsfile"); + TableSchema tableSchema = + new TableSchema( + "load_min_max", + Arrays.asList("id1", "s1"), + Arrays.asList(TSDataType.STRING, TSDataType.INT32), + Arrays.asList(ColumnCategory.ID, ColumnCategory.MEASUREMENT)); + + try (ITsFileWriter writer = + new TsFileWriterBuilder().file(file).tableSchema(tableSchema).build()) { + Tablet tablet = + new Tablet( + Arrays.asList("id1", "s1"), Arrays.asList(TSDataType.STRING, TSDataType.INT32)); + tablet.addTimestamp(0, Long.MIN_VALUE); + tablet.addTimestamp(1, Long.MAX_VALUE); + tablet.addValue(0, 0, "d1"); + tablet.addValue(1, 0, "d1"); + tablet.addValue(0, 1, 1); + tablet.addValue(1, 1, 1); + writer.write(tablet); + } + + try (ITableSession session = EnvFactory.getEnv().getTableSessionConnection()) { + session.executeNonQueryStatement("USE db1"); + try { + session.executeNonQueryStatement( + "SET CONFIGURATION timestamp_precision_check_enabled='false'"); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + session.executeNonQueryStatement("LOAD \'" + file.getAbsolutePath() + "\'"); + + SessionDataSet dataSet = session.executeQueryStatement("SELECT * FROM load_min_max"); + RowRecord record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getFields().get(0).getLongV()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getFields().get(0).getLongV()); + assertFalse(dataSet.hasNext()); + } finally { + try (ITableSession session = EnvFactory.getEnv().getTableSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION timestamp_precision_check_enabled='false'"); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + } + file.delete(); + } + } } diff --git a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java index f7fa3e3345b1..2fd4136aa51b 100644 --- a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java @@ -79,7 +79,7 @@ import static org.junit.Assert.assertTrue; import static org.junit.Assert.fail; -@SuppressWarnings("ThrowFromFinallyBlock") +@SuppressWarnings({"ThrowFromFinallyBlock", "ResultOfMethodCallIgnored"}) @RunWith(IoTDBTestRunner.class) public class IoTDBSessionSimpleIT { @@ -1925,7 +1925,7 @@ record = dataSet.next(); @Test @Category({LocalStandaloneIT.class, ClusterIT.class}) - public void loadMinMaxTimeTest() + public void loadMinMaxTimeNonAlignedTest() throws IoTDBConnectionException, StatementExecutionException, IOException, @@ -1975,4 +1975,58 @@ record = dataSet.next(); file.delete(); } } + + @Test + @Category({LocalStandaloneIT.class, ClusterIT.class}) + public void loadMinMaxTimeAlignedTest() + throws IoTDBConnectionException, + StatementExecutionException, + IOException, + WriteProcessException { + File file = new File("target", "test.tsfile"); + try (TsFileWriter writer = new TsFileWriter(file)) { + IDeviceID deviceID = Factory.DEFAULT_FACTORY.create("root.testLoadMinMaxAligned.d1"); + writer.registerAlignedTimeseries( + deviceID, Collections.singletonList(new MeasurementSchema("s1", TSDataType.INT32))); + TSRecord record = new TSRecord(deviceID, Long.MIN_VALUE); + record.addPoint("s1", 1); + writer.writeRecord(record); + record.setTime(Long.MAX_VALUE); + writer.writeRecord(record); + } + + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION \"timestamp_precision_check_enabled\"=\"false\""); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + session.executeNonQueryStatement("LOAD \"" + file.getAbsolutePath() + "\""); + + SessionDataSet dataSet = + session.executeQueryStatement("SELECT * FROM root.testLoadMinMaxAligned.d1"); + RowRecord record = dataSet.next(); + assertEquals(Long.MIN_VALUE, record.getTimestamp()); + record = dataSet.next(); + assertEquals(Long.MAX_VALUE, record.getTimestamp()); + assertFalse(dataSet.hasNext()); + } finally { + try (ISession session = EnvFactory.getEnv().getSessionConnection()) { + try { + session.executeNonQueryStatement( + "SET CONFIGURATION \"timestamp_precision_check_enabled\"=\"true\""); + } catch (StatementExecutionException e) { + // run in IDE will trigger this, ignore it + if (!e.getMessage().contains("Unable to find the configuration file")) { + throw e; + } + } + } + file.delete(); + } + } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java index 7d8fb91986d4..aca2d76496b5 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java @@ -63,11 +63,11 @@ import java.util.HashSet; import java.util.List; import java.util.Map; -import java.util.Optional; import java.util.Set; import static org.apache.tsfile.read.reader.chunk.ChunkReader.decryptAndUncompressPageData; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class RepairDataFileScanUtil { private static final Logger logger = LoggerFactory.getLogger(RepairDataFileScanUtil.class); private final TsFileResource resource; @@ -110,11 +110,11 @@ public void scanTsFile(boolean checkTsFileResource) { resource.isSeq() ? CompactionType.INNER_SEQ_COMPACTION : CompactionType.INNER_UNSEQ_COMPACTION)) { - TsFileDeviceIterator deviceInfileIterator = reader.getAllDevicesIteratorWithIsAligned(); + TsFileDeviceIterator deviceInFileIterator = reader.getAllDevicesIteratorWithIsAligned(); Set deviceIdsInTimeIndex = checkTsFileResource ? new HashSet<>(timeIndex.getDevices()) : Collections.emptySet(); - while (deviceInfileIterator.hasNext()) { - Pair deviceIsAlignedPair = deviceInfileIterator.next(); + while (deviceInFileIterator.hasNext()) { + Pair deviceIsAlignedPair = deviceInFileIterator.next(); IDeviceID deviceInfile = deviceIsAlignedPair.getLeft(); if (checkTsFileResource) { if (!deviceIdsInTimeIndex.contains(deviceInfile)) { @@ -124,15 +124,15 @@ public void scanTsFile(boolean checkTsFileResource) { deviceIdsInTimeIndex.remove(deviceInfile); } MetadataIndexNode metadataIndexNode = - deviceInfileIterator.getFirstMeasurementNodeOfCurrentDevice(); + deviceInFileIterator.getFirstMeasurementNodeOfCurrentDevice(); - Optional startTime = timeIndex.getStartTime(deviceInfile); - Optional endTime = timeIndex.getEndTime(deviceInfile); - if (!startTime.isPresent() || !endTime.isPresent()) { - continue; - } + // presence checked above TimeRange deviceTimeRangeInResource = - checkTsFileResource ? new TimeRange(startTime.get(), endTime.get()) : null; + checkTsFileResource + ? new TimeRange( + timeIndex.getStartTime(deviceInfile).get(), + timeIndex.getEndTime(deviceInfile).get()) + : null; boolean isAligned = deviceIsAlignedPair.getRight(); if (isAligned) { checkAlignedDeviceSeries( diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/impl/SettleSelectorImpl.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/impl/SettleSelectorImpl.java index 67d66e204b62..a09f298dcd6c 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/impl/SettleSelectorImpl.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/impl/SettleSelectorImpl.java @@ -238,8 +238,10 @@ private FileDirtyInfo selectFileBaseOnDirtyData(TsFileResource resource) throws // overflow, like 100 - Long.MIN outdatedTimeDiff = Long.MAX_VALUE; } + long ttlThreshold = 3 * ttl > ttl ? ttl : Long.MAX_VALUE; hasExpiredTooLong = - hasExpiredTooLong || outdatedTimeDiff > Math.min(config.getMaxExpiredTime(), 3 * ttl); + hasExpiredTooLong + || outdatedTimeDiff > Math.min(config.getMaxExpiredTime(), ttlThreshold); } // else hasExpiredTooLong unchanged } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java index a90fb80511fa..9274115bc47c 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/TsFilePlanRedoer.java @@ -86,7 +86,7 @@ void redoInsert(InsertNode node) throws WriteProcessException { if (!node.hasValidMeasurements()) { return; } - if (tsFileResource != null && !tsFileResource.definitelyNotContains(node.getDeviceID())) { + if (tsFileResource != null) { // orders of insert node is guaranteed by storage engine, just check time in the file // the last chunk group may contain the same data with the logs, ignore such logs in seq file Optional lastEndTime = tsFileResource.getEndTime(node.getDeviceID()); @@ -128,7 +128,7 @@ void redoInsertRows(InsertRowsNode insertRowsNode) { if (!node.hasValidMeasurements()) { continue; } - if (tsFileResource != null && !tsFileResource.definitelyNotContains(node.getDeviceID())) { + if (tsFileResource != null) { // orders of insert node is guaranteed by storage engine, just check time in the file // the last chunk group may contain the same data with the logs, ignore such logs in seq // file diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java index 225d87a86bc4..ca7493d6ca06 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/AlignedChunkData.java @@ -240,7 +240,7 @@ public void writeDecodeValuePage( pageNumbers.set(pageNumbers.size() - 1, pageNumbers.get(pageNumbers.size() - 1) + 1); final long startTime = timePartitionSlot.getStartTime(); // beware of overflow - long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + long endTime = startTime + TimePartitionUtils.getTimePartitionInterval() - 1; if (endTime <= startTime) { endTime = Long.MAX_VALUE; } @@ -251,7 +251,7 @@ public void writeDecodeValuePage( satisfiedLengthQueue.offer(satisfiedLength); for (int i = 0; i < times.length; i++) { - if (times[i] >= endTime) { + if (times[i] > endTime) { break; } if (times[i] >= startTime) { diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java index d79fdeb09e05..5a8e22aa8e2a 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/load/splitter/BatchedAlignedValueChunkData.java @@ -71,7 +71,7 @@ public void writeDecodeValuePage(long[] times, TsPrimitiveType[] values, TSDataT pageNumbers.set(pageNumbers.size() - 1, pageNumbers.get(pageNumbers.size() - 1) + 1); final long startTime = timePartitionSlot.getStartTime(); // beware of overflow - long endTime = startTime + TimePartitionUtils.getTimePartitionInterval(); + long endTime = startTime + TimePartitionUtils.getTimePartitionInterval() - 1; if (endTime <= startTime) { endTime = Long.MAX_VALUE; } @@ -85,7 +85,7 @@ public void writeDecodeValuePage(long[] times, TsPrimitiveType[] values, TSDataT // its start time and end time. long pageStartTime = Long.MAX_VALUE, pageEndTime = Long.MIN_VALUE; for (int i = 0; i < times.length; i++) { - if (times[i] >= endTime) { + if (times[i] > endTime) { break; } if (times[i] >= startTime) { diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/CompactionDataTypeNotMatchTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/CompactionDataTypeNotMatchTest.java index 4e7810b65e4e..ff12a13c6dac 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/CompactionDataTypeNotMatchTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/CompactionDataTypeNotMatchTest.java @@ -48,6 +48,7 @@ import java.util.ArrayList; import java.util.List; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class CompactionDataTypeNotMatchTest extends AbstractCompactionTest { private final String oldThreadName = Thread.currentThread().getName(); private final IDeviceID device = @@ -75,7 +76,8 @@ public void testCompactNonAlignedSeriesWithReadChunkCompactionPerformer() 0, tsFileManager, seqResources, true, new ReadChunkCompactionPerformer(), 0); Assert.assertTrue(task.start()); TsFileResourceUtils.validateTsFileDataCorrectness(tsFileManager.getTsFileList(true).get(0)); - Assert.assertEquals(2, tsFileManager.getTsFileList(true).get(0).getStartTime(device)); + Assert.assertEquals( + 2, ((long) tsFileManager.getTsFileList(true).get(0).getStartTime(device).get())); } @Test @@ -87,7 +89,8 @@ public void testCompactNonAlignedSeriesWithFastCompactionPerformer() 0, tsFileManager, seqResources, true, new FastCompactionPerformer(false), 0); Assert.assertTrue(task.start()); TsFileResourceUtils.validateTsFileDataCorrectness(tsFileManager.getTsFileList(true).get(0)); - Assert.assertEquals(2, tsFileManager.getTsFileList(true).get(0).getStartTime(device)); + Assert.assertEquals( + 2, ((long) tsFileManager.getTsFileList(true).get(0).getStartTime(device).get())); } @Test @@ -99,7 +102,8 @@ public void testCompactNonAlignedSeriesWithReadPointCompactionPerformer() 0, tsFileManager, seqResources, true, new ReadPointCompactionPerformer(), 0); Assert.assertTrue(task.start()); TsFileResourceUtils.validateTsFileDataCorrectness(tsFileManager.getTsFileList(true).get(0)); - Assert.assertEquals(2, tsFileManager.getTsFileList(true).get(0).getStartTime(device)); + Assert.assertEquals( + 2, ((long) tsFileManager.getTsFileList(true).get(0).getStartTime(device).get())); } @Test @@ -111,7 +115,8 @@ public void testCompactAlignedSeriesWithReadChunkCompactionPerformer() 0, tsFileManager, seqResources, true, new ReadChunkCompactionPerformer(), 0); Assert.assertTrue(task.start()); TsFileResourceUtils.validateTsFileDataCorrectness(tsFileManager.getTsFileList(true).get(0)); - Assert.assertEquals(2, tsFileManager.getTsFileList(true).get(0).getStartTime(device)); + Assert.assertEquals( + 2, ((long) tsFileManager.getTsFileList(true).get(0).getStartTime(device).get())); } @Test @@ -123,7 +128,8 @@ public void testCompactAlignedSeriesWithFastCompactionPerformer() 0, tsFileManager, seqResources, true, new FastCompactionPerformer(false), 0); Assert.assertTrue(task.start()); TsFileResourceUtils.validateTsFileDataCorrectness(tsFileManager.getTsFileList(true).get(0)); - Assert.assertEquals(2, tsFileManager.getTsFileList(true).get(0).getStartTime(device)); + Assert.assertEquals( + 2, ((long) tsFileManager.getTsFileList(true).get(0).getStartTime(device).get())); } @Test @@ -135,7 +141,8 @@ public void testCompactAlignedSeriesWithReadPointCompactionPerformer() 0, tsFileManager, seqResources, true, new ReadPointCompactionPerformer(), 0); Assert.assertTrue(task.start()); TsFileResourceUtils.validateTsFileDataCorrectness(tsFileManager.getTsFileList(true).get(0)); - Assert.assertEquals(2, tsFileManager.getTsFileList(true).get(0).getStartTime(device)); + Assert.assertEquals( + 2, ((long) tsFileManager.getTsFileList(true).get(0).getStartTime(device).get())); } private void generateDataTypeNotMatchFilesWithNonAlignedSeries() diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastCompactionPerformerWithEmptyPageTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastCompactionPerformerWithEmptyPageTest.java index 6a4b57f30801..7e9c25316a54 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastCompactionPerformerWithEmptyPageTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastCompactionPerformerWithEmptyPageTest.java @@ -46,6 +46,7 @@ import java.util.List; import java.util.Map; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class FastCompactionPerformerWithEmptyPageTest extends AbstractCompactionTest { int oldAlignedSeriesCompactionBatchSize; @@ -123,8 +124,8 @@ public void test1() throws IOException, IllegalPathException { } TsFileResource result = tsFileManager.getTsFileList(true).get(0); result.buildDeviceTimeIndex(); - Assert.assertEquals(20, result.getStartTime(device)); - Assert.assertEquals(50, result.getEndTime(device)); + Assert.assertEquals(20, ((long) result.getStartTime(device).get())); + Assert.assertEquals(50, ((long) result.getEndTime(device).get())); validateSeqFiles(true); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastInnerCompactionPerformerTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastInnerCompactionPerformerTest.java index 8c9088f5cebc..a0d84c1e8104 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastInnerCompactionPerformerTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/FastInnerCompactionPerformerTest.java @@ -43,6 +43,7 @@ import org.apache.tsfile.file.metadata.ChunkMetadata; import org.apache.tsfile.file.metadata.IChunkMetadata; import org.apache.tsfile.file.metadata.IDeviceID; +import org.apache.tsfile.file.metadata.IDeviceID.Factory; import org.apache.tsfile.file.metadata.enums.CompressionType; import org.apache.tsfile.file.metadata.enums.TSEncoding; import org.apache.tsfile.read.TimeValuePair; @@ -75,6 +76,7 @@ import static org.apache.iotdb.commons.conf.IoTDBConstant.PATH_SEPARATOR; import static org.junit.Assert.assertEquals; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class FastInnerCompactionPerformerTest extends AbstractCompactionTest { private boolean enableUnseqSpaceCompaction = @@ -243,47 +245,53 @@ public void testSeqInnerSpaceCompactionWithDifferentTimeseries() throws Exceptio assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d0"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d0")) + .get())); assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d1"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d1")) + .get())); assertEquals( 250, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d2"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d2")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d3"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d3")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d4"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d4")) + .get())); for (int i = 0; i < 5; i++) { assertEquals( 749, - targetResources - .get(0) - .getEndTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i))); + ((long) + targetResources + .get(0) + .getEndTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i)) + .get())); } for (int i = 0; i < 5; i++) { @@ -408,47 +416,53 @@ public void testSeqInnerSpaceCompactionWithFileTimeIndex() throws Exception { assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d0"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d0")) + .get())); assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d1"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d1")) + .get())); assertEquals( 250, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d2"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d2")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d3"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d3")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d4"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d4")) + .get())); for (int i = 0; i < 5; i++) { assertEquals( 749, - targetResources - .get(0) - .getEndTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i))); + ((long) + targetResources + .get(0) + .getEndTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i)) + .get())); } for (int i = 0; i < 5; i++) { diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResourceTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResourceTest.java index 65bf0bbf85bd..1fa5a25d17dd 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResourceTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/tsfile/TsFileResourceTest.java @@ -25,6 +25,7 @@ import org.apache.commons.io.FileUtils; import org.apache.tsfile.file.metadata.IDeviceID; +import org.apache.tsfile.file.metadata.IDeviceID.Factory; import org.junit.After; import org.junit.Assert; import org.junit.Before; @@ -36,6 +37,7 @@ import java.util.Map; import java.util.stream.IntStream; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class TsFileResourceTest { private final File file = new File( @@ -93,11 +95,13 @@ public void testDegradeAndFileTimeIndex() { Assert.assertEquals(deviceToIndex.keySet(), tsFileResource.getDevices()); for (int i = 0; i < DEVICE_NUM; i++) { Assert.assertEquals( - tsFileResource.getStartTime(IDeviceID.Factory.DEFAULT_FACTORY.create("root.sg1.d" + i)), - 0); + 0, + ((long) + tsFileResource.getStartTime(Factory.DEFAULT_FACTORY.create("root.sg1.d" + i)).get())); Assert.assertEquals( - tsFileResource.getEndTime(IDeviceID.Factory.DEFAULT_FACTORY.create("root.sg1.d" + i)), - DEVICE_NUM); + DEVICE_NUM, + ((long) + tsFileResource.getEndTime(Factory.DEFAULT_FACTORY.create("root.sg1.d" + i)).get())); } } } diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/WALRecoverManagerTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/WALRecoverManagerTest.java index e4f82b5b367e..248269062837 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/WALRecoverManagerTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/WALRecoverManagerTest.java @@ -89,6 +89,7 @@ import static org.junit.Assert.assertTrue; import static org.junit.Assert.fail; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class WALRecoverManagerTest { private static final IoTDBConfig config = IoTDBDescriptor.getInstance().getConfig(); private static final CommonConfig commonConfig = CommonDescriptor.getInstance().getConfig(); @@ -327,10 +328,10 @@ private void recoverAndCheck() throws Exception { assertEquals(15, chunk.getChunkStatistic().getEndTime()); reader.close(); // check .resource file in memory - assertEquals(1, tsFileWithWALResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileWithWALResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileWithWALResource.getStartTime(DEVICE2_NAME)); - assertEquals(15, tsFileWithWALResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileWithWALResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileWithWALResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileWithWALResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(15, ((long) tsFileWithWALResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(new File(FILE_WITH_WAL_NAME).exists()); assertTrue(new File(FILE_WITH_WAL_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); @@ -352,10 +353,10 @@ private void recoverAndCheck() throws Exception { assertEquals(3, chunk.getChunkStatistic().getEndTime()); reader.close(); // check .resource file in memory - assertEquals(1, tsFileWithoutWALResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileWithoutWALResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileWithoutWALResource.getStartTime(DEVICE2_NAME)); - assertEquals(3, tsFileWithoutWALResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileWithoutWALResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileWithoutWALResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileWithoutWALResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(3, ((long) tsFileWithoutWALResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(new File(FILE_WITHOUT_WAL_NAME).exists()); assertTrue(new File(FILE_WITHOUT_WAL_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); @@ -437,8 +438,8 @@ private void recoverFromOldWalAndCheck(boolean withBrokenTsFile) throws Exceptio assertEquals(15, chunk.getChunkStatistic().getEndTime()); reader.close(); // check .resource file in memory - assertEquals(4, tsFileWithWALResource.getStartTime(DEVICE2_NAME)); - assertEquals(15, tsFileWithWALResource.getEndTime(DEVICE2_NAME)); + assertEquals(4, ((long) tsFileWithWALResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(15, ((long) tsFileWithWALResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(fileWithWAL.exists()); assertTrue(new File(fileWithWAL.getPath().concat(TsFileResource.RESOURCE_SUFFIX)).exists()); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/SealedTsFileRecoverPerformerTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/SealedTsFileRecoverPerformerTest.java index f1de11549e23..99563ec91761 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/SealedTsFileRecoverPerformerTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/SealedTsFileRecoverPerformerTest.java @@ -55,6 +55,7 @@ import static org.junit.Assert.assertNotNull; import static org.junit.Assert.assertTrue; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class SealedTsFileRecoverPerformerTest { private static final String SG_NAME = "root.recover_sg"; private static final IDeviceID DEVICE1_NAME = @@ -113,10 +114,10 @@ public void testCompleteFileWithResource() throws Exception { assertNotNull(chunkMetadataList); reader.close(); // check .resource file in memory - assertEquals(1, tsFileResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileResource.getStartTime(DEVICE2_NAME)); - assertEquals(4, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(4, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(file.exists()); assertTrue(new File(FILE_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); @@ -155,10 +156,10 @@ public void testCompleteFileWithBrokenResource() throws Exception { assertNotNull(chunkMetadataList); reader.close(); // check .resource file in memory - assertEquals(1, tsFileResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileResource.getStartTime(DEVICE2_NAME)); - assertEquals(4, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(4, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(file.exists()); assertTrue(new File(FILE_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); @@ -192,11 +193,11 @@ public void testCompleteFileWithoutResource() throws Exception { assertNotNull(chunkMetadataList); reader.close(); // check .resource file in memory - assertEquals(1, tsFileResource.getStartTime(DEVICE1_NAME)); + assertEquals(1, ((long) tsFileResource.getStartTime(DEVICE1_NAME).get())); - assertEquals(2, tsFileResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileResource.getStartTime(DEVICE2_NAME)); - assertEquals(4, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(2, ((long) tsFileResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(4, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(file.exists()); assertTrue(new File(FILE_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); @@ -262,10 +263,10 @@ public void testCrashedFile() throws Exception { assertEquals(3, chunk.getChunkStatistic().getEndTime()); reader.close(); // check .resource file in memory - assertEquals(1, tsFileResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileResource.getStartTime(DEVICE2_NAME)); - assertEquals(3, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(3, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(file.exists()); assertTrue(new File(FILE_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformerTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformerTest.java index ac8a187d1875..3046ddae50df 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformerTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/wal/recover/file/UnsealedTsFileRecoverPerformerTest.java @@ -85,6 +85,7 @@ import static org.junit.Assert.assertNotNull; import static org.junit.Assert.assertTrue; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class UnsealedTsFileRecoverPerformerTest { private static final IoTDBConfig config = IoTDBDescriptor.getInstance().getConfig(); @@ -149,7 +150,7 @@ public void testRedoInsertPlan() throws Exception { recoverPerformer.startRecovery(); assertTrue(recoverPerformer.hasCrashed()); assertTrue(recoverPerformer.canWrite()); - assertEquals(3, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(3, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); recoverPerformer.redoLog(walEntry); @@ -173,10 +174,10 @@ public void testRedoInsertPlan() throws Exception { assertEquals(4, chunk.getChunkStatistic().getEndTime()); reader.close(); // check .resource file in memory - assertEquals(1, tsFileResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileResource.getStartTime(DEVICE2_NAME)); - assertEquals(4, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(4, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(file.exists()); assertTrue(new File(FILE_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); @@ -245,7 +246,7 @@ public void testRedoDeletePlan() throws Exception { recoverPerformer.startRecovery(); assertTrue(recoverPerformer.hasCrashed()); assertTrue(recoverPerformer.canWrite()); - assertEquals(3, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(3, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); recoverPerformer.redoLog(walEntry); @@ -267,10 +268,10 @@ public void testRedoDeletePlan() throws Exception { assertEquals(3, chunk.getChunkStatistic().getEndTime()); reader.close(); // check .resource file in memory - assertEquals(1, tsFileResource.getStartTime(DEVICE1_NAME)); - assertEquals(2, tsFileResource.getEndTime(DEVICE1_NAME)); - assertEquals(3, tsFileResource.getStartTime(DEVICE2_NAME)); - assertEquals(3, tsFileResource.getEndTime(DEVICE2_NAME)); + assertEquals(1, ((long) tsFileResource.getStartTime(DEVICE1_NAME).get())); + assertEquals(2, ((long) tsFileResource.getEndTime(DEVICE1_NAME).get())); + assertEquals(3, ((long) tsFileResource.getStartTime(DEVICE2_NAME).get())); + assertEquals(3, ((long) tsFileResource.getEndTime(DEVICE2_NAME).get())); // check file existence assertTrue(file.exists()); assertTrue(new File(FILE_NAME.concat(TsFileResource.RESOURCE_SUFFIX)).exists()); diff --git a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties index db2835acfc52..c51f948ad250 100644 --- a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties +++ b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties @@ -70,19 +70,3 @@ cn_metric_prometheus_reporter_port=9091 # dn_metric_reporter_list= dn_metric_prometheus_reporter_port=9092 -#Wed Dec 18 17:50:40 HKT 2024 -#timestamp_precision_check_enabled=false -#Wed Dec 18 18:05:55 HKT 2024 -#timestamp_precision_check_enabled=true -#Wed Dec 18 18:17:25 HKT 2024 -#timestamp_precision_check_enabled=false -#Wed Dec 18 18:17:25 HKT 2024 -#timestamp_precision_check_enabled=true -#Wed Dec 18 18:18:03 HKT 2024 -#timestamp_precision_check_enabled=false -#Wed Dec 18 18:19:27 HKT 2024 -#timestamp_precision_check_enabled=true -#Wed Dec 18 18:19:55 HKT 2024 -#timestamp_precision_check_enabled=false -#Wed Dec 18 18:55:14 HKT 2024 -timestamp_precision_check_enabled=true diff --git a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template index 978148101442..45b6895cd96c 100644 --- a/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template +++ b/iotdb-core/node-commons/src/assembly/resources/conf/iotdb-system.properties.template @@ -1075,7 +1075,7 @@ timestamp_precision=ms # When the timestamp precision check is enabled, the timestamps those are over 13 digits for ms precision, or over 16 digits for us precision are not allowed to be inserted. # For all precisions, ms, us and ns, the timestamps cannot exceed the range of [-9223372036854775808, 9223372036854775807], regardless of whether the check is enabled or not. -# effectiveMode: restart +# effectiveMode: hot_reload # Datatype: Boolean timestamp_precision_check_enabled=true From 057b39977b995e4b51132bb01e3417142a446e27 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Thu, 19 Dec 2024 18:19:30 +0800 Subject: [PATCH 07/12] fix test --- .../ReadPointCompactionPerformerTest.java | 134 ++++++++++-------- 1 file changed, 74 insertions(+), 60 deletions(-) diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/ReadPointCompactionPerformerTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/ReadPointCompactionPerformerTest.java index 8f603332a123..65163221af4d 100644 --- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/ReadPointCompactionPerformerTest.java +++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/storageengine/dataregion/compaction/ReadPointCompactionPerformerTest.java @@ -43,6 +43,7 @@ import org.apache.tsfile.enums.TSDataType; import org.apache.tsfile.exception.write.WriteProcessException; import org.apache.tsfile.file.metadata.IDeviceID; +import org.apache.tsfile.file.metadata.IDeviceID.Factory; import org.apache.tsfile.read.common.IBatchDataIterator; import org.apache.tsfile.read.common.block.TsBlock; import org.apache.tsfile.utils.Pair; @@ -66,6 +67,7 @@ import static org.apache.iotdb.commons.conf.IoTDBConstant.PATH_SEPARATOR; import static org.junit.Assert.assertEquals; +@SuppressWarnings("OptionalGetWithoutIsPresent") public class ReadPointCompactionPerformerTest extends AbstractCompactionTest { private final String oldThreadName = Thread.currentThread().getName(); @@ -228,47 +230,53 @@ public void testSeqInnerSpaceCompactionWithDifferentTimeseries() throws Exceptio targetResources, CompactionTaskType.INNER_SEQ, COMPACTION_TEST_SG); assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d0"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d0")) + .get())); assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d1"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d1")) + .get())); assertEquals( 250, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d2"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d2")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d3"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d3")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d4"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d4")) + .get())); for (int i = 0; i < 5; i++) { assertEquals( 749, - targetResources - .get(0) - .getEndTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i))); + ((long) + targetResources + .get(0) + .getEndTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i)) + .get())); } for (int i = 0; i < 5; i++) { @@ -390,47 +398,53 @@ public void testSeqInnerSpaceCompactionWithFileTimeIndex() throws Exception { targetResources, CompactionTaskType.INNER_SEQ, COMPACTION_TEST_SG); assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d0"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d0")) + .get())); assertEquals( 0, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d1"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d1")) + .get())); assertEquals( 250, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d2"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d2")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d3"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d3")) + .get())); assertEquals( 600, - targetResources - .get(0) - .getStartTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d4"))); + ((long) + targetResources + .get(0) + .getStartTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d4")) + .get())); for (int i = 0; i < 5; i++) { assertEquals( 749, - targetResources - .get(0) - .getEndTime( - IDeviceID.Factory.DEFAULT_FACTORY.create( - COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i))); + ((long) + targetResources + .get(0) + .getEndTime( + Factory.DEFAULT_FACTORY.create(COMPACTION_TEST_SG + PATH_SEPARATOR + "d" + i)) + .get())); } for (int i = 0; i < 5; i++) { From 5c4214d17110a586c0f50d4a54997e27c3b638d3 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Fri, 20 Dec 2024 11:31:40 +0800 Subject: [PATCH 08/12] avoid test confilct --- .../java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java index 2fd4136aa51b..8769a26de55e 100644 --- a/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/session/it/IoTDBSessionSimpleIT.java @@ -756,7 +756,7 @@ public void createWrongTimeSeriesTest() { LOGGER.error("", e); } - final SessionDataSet dataSet = session.executeQueryStatement("SHOW TIMESERIES"); + final SessionDataSet dataSet = session.executeQueryStatement("SHOW TIMESERIES root.sg.**"); assertFalse(dataSet.hasNext()); session.deleteStorageGroup(storageGroup); From 6a67574a2905427c71efae7769c1098fa124458c Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Sat, 21 Dec 2024 14:31:34 +0800 Subject: [PATCH 09/12] fix review --- .../AlignedSeriesCompactionExecutor.java | 4 ++- ...dChunkAlignedSeriesCompactionExecutor.java | 6 +++-- .../SingleSeriesCompactionExecutor.java | 26 ++++++++++++------- .../writer/AbstractCompactionWriter.java | 5 ++-- .../writer/AbstractCrossCompactionWriter.java | 1 + .../writer/AbstractInnerCompactionWriter.java | 1 + .../writer/FastCrossCompactionWriter.java | 6 +++++ .../writer/FastInnerCompactionWriter.java | 6 +++++ .../ReadPointCrossCompactionWriter.java | 1 + .../repair/RepairDataFileScanUtil.java | 13 ++++++---- 10 files changed, 50 insertions(+), 19 deletions(-) diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/AlignedSeriesCompactionExecutor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/AlignedSeriesCompactionExecutor.java index 02628027ea02..1abef8e85d65 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/AlignedSeriesCompactionExecutor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/AlignedSeriesCompactionExecutor.java @@ -61,6 +61,7 @@ public class AlignedSeriesCompactionExecutor { private final List schemaList; private long remainingPointInChunkWriter = 0L; private final CompactionTaskSummary summary; + private boolean lastWriteTimestampSet = false; private long lastWriteTimestamp = Long.MIN_VALUE; private final long chunkSizeThreshold = @@ -210,11 +211,12 @@ private void flushChunkWriterIfLargeEnough() throws IOException { } private void checkAndUpdatePreviousTimestamp(long currentWritingTimestamp) { - if (currentWritingTimestamp <= lastWriteTimestamp) { + if (lastWriteTimestampSet && currentWritingTimestamp <= lastWriteTimestamp) { throw new CompactionLastTimeCheckFailedException( device.toString(), currentWritingTimestamp, lastWriteTimestamp); } else { lastWriteTimestamp = currentWritingTimestamp; + lastWriteTimestampSet = true; } } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/ReadChunkAlignedSeriesCompactionExecutor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/ReadChunkAlignedSeriesCompactionExecutor.java index 12c205c472ed..a22f9627ed01 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/ReadChunkAlignedSeriesCompactionExecutor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/ReadChunkAlignedSeriesCompactionExecutor.java @@ -80,7 +80,8 @@ public class ReadChunkAlignedSeriesCompactionExecutor { protected final CompactionTaskSummary summary; protected final boolean ignoreAllNullRows; - private long lastWriteTimestamp = Long.MIN_VALUE; + private long lastWriteTimestamp; + private boolean lastWriteTimestampSet = false; public ReadChunkAlignedSeriesCompactionExecutor( IDeviceID device, @@ -416,11 +417,12 @@ protected IPointReader getPointReader( } protected void checkAndUpdatePreviousTimestamp(long currentWritingTimestamp) { - if (currentWritingTimestamp <= lastWriteTimestamp) { + if (lastWriteTimestampSet && currentWritingTimestamp <= lastWriteTimestamp) { throw new CompactionLastTimeCheckFailedException( device.toString(), currentWritingTimestamp, lastWriteTimestamp); } else { lastWriteTimestamp = currentWritingTimestamp; + lastWriteTimestampSet = true; } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java index 68c176591ca4..69ce89d5af37 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/executor/readchunk/SingleSeriesCompactionExecutor.java @@ -57,9 +57,12 @@ public class SingleSeriesCompactionExecutor { private Chunk cachedChunk; private ChunkMetadata cachedChunkMetadata; // record the min time and max time to update the target resource - private Long minStartTimestamp = null; - private Long maxEndTimestamp = null; - private Long lastWriteTimestamp = null; + private boolean minStartTimestampSet = false; + private long minStartTimestamp; + private boolean maxEndTimestampSet = false; + private long maxEndTimestamp; + private boolean lastWriteTimestampSet = false; + private long lastWriteTimestamp; private long pointCountInChunkWriter = 0; private final CompactionTaskSummary summary; @@ -158,7 +161,7 @@ public void execute() throws IOException { flushChunkWriter(); } fileWriter.checkMetadataSizeAndMayFlush(); - if (minStartTimestamp != null) { + if (minStartTimestampSet) { targetResource.updateStartTime(device, minStartTimestamp); targetResource.updateEndTime(device, maxEndTimestamp); } @@ -271,11 +274,13 @@ private void writeChunkIntoChunkWriter(Chunk chunk) throws IOException { TimeValuePair timeValuePair = batchIterator.nextTimeValuePair(); checkAndUpdatePreviousTimestamp(timeValuePair.getTimestamp()); writeTimeAndValueToChunkWriter(timeValuePair); - if (maxEndTimestamp == null || timeValuePair.getTimestamp() > maxEndTimestamp) { + if (!maxEndTimestampSet || timeValuePair.getTimestamp() > maxEndTimestamp) { maxEndTimestamp = timeValuePair.getTimestamp(); + maxEndTimestampSet = true; } - if (minStartTimestamp == null || timeValuePair.getTimestamp() < minStartTimestamp) { + if (!minStartTimestampSet || timeValuePair.getTimestamp() < minStartTimestamp) { minStartTimestamp = timeValuePair.getTimestamp(); + minStartTimestampSet = true; } } } @@ -338,11 +343,13 @@ private void writeTimeAndValueToChunkWriter(TimeValuePair timeValuePair) { private void flushChunkToFileWriter(Chunk chunk, ChunkMetadata chunkMetadata) throws IOException { checkAndUpdatePreviousTimestamp(chunkMetadata.getStartTime()); - if (minStartTimestamp == null || chunkMetadata.getStartTime() < minStartTimestamp) { + if (!minStartTimestampSet || chunkMetadata.getStartTime() < minStartTimestamp) { minStartTimestamp = chunkMetadata.getStartTime(); + minStartTimestampSet = true; } - if (maxEndTimestamp == null || chunkMetadata.getEndTime() > maxEndTimestamp) { + if (!maxEndTimestampSet || chunkMetadata.getEndTime() > maxEndTimestamp) { maxEndTimestamp = chunkMetadata.getEndTime(); + maxEndTimestampSet = true; } fileWriter.writeChunk(chunk, chunkMetadata); } @@ -370,11 +377,12 @@ private void flushChunkWriter() throws IOException { } private void checkAndUpdatePreviousTimestamp(long currentWritingTimestamp) { - if (lastWriteTimestamp != null && currentWritingTimestamp <= lastWriteTimestamp) { + if (lastWriteTimestampSet && currentWritingTimestamp <= lastWriteTimestamp) { throw new CompactionLastTimeCheckFailedException( device, measurement, currentWritingTimestamp, lastWriteTimestamp); } else { lastWriteTimestamp = currentWritingTimestamp; + lastWriteTimestampSet = true; } } } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCompactionWriter.java index 42e89fd8bbb9..364e21e22737 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCompactionWriter.java @@ -52,6 +52,7 @@ public abstract class AbstractCompactionWriter implements AutoCloseable { // check if there is unseq error point during writing protected long[] lastTime = new long[subTaskNum]; + protected boolean[] lastTimeSet = new boolean[subTaskNum]; // Each sub task has its own chunk writer. // The index of the array corresponds to subTaskId. @@ -104,7 +105,7 @@ public abstract class AbstractCompactionWriter implements AutoCloseable { public void startMeasurement(String measurement, IChunkWriter chunkWriter, int subTaskId) { lastCheckIndex = 0; - lastTime[subTaskId] = Long.MIN_VALUE; + lastTimeSet[subTaskId] = false; chunkWriters[subTaskId] = chunkWriter; measurementId[subTaskId] = measurement; } @@ -313,7 +314,7 @@ protected long getChunkSize(Chunk chunk) { } protected void checkPreviousTimestamp(long currentWritingTimestamp, int subTaskId) { - if (currentWritingTimestamp <= lastTime[subTaskId]) { + if (lastTimeSet[subTaskId] && currentWritingTimestamp <= lastTime[subTaskId]) { throw new CompactionLastTimeCheckFailedException( deviceId.toString() + IoTDBConstant.PATH_SEPARATOR + measurementId[subTaskId], currentWritingTimestamp, diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java index efab75d42795..9d074b7fd46f 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractCrossCompactionWriter.java @@ -137,6 +137,7 @@ public void write(TimeValuePair timeValuePair, int subTaskId) throws IOException isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = timestamp; + lastTimeSet[subTaskId] = true; } /** Write data in batch, only used for aligned device. */ diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractInnerCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractInnerCompactionWriter.java index a06d3f6a63c8..9d5e6f498254 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractInnerCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/AbstractInnerCompactionWriter.java @@ -121,6 +121,7 @@ public void write(TimeValuePair timeValuePair, int subTaskId) throws IOException chunkPointNumArray[subTaskId]++; checkChunkSizeAndMayOpenANewChunk(fileWriter, chunkWriters[subTaskId], subTaskId); lastTime[subTaskId] = timeValuePair.getTimestamp(); + lastTimeSet[subTaskId] = true; } @Override diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastCrossCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastCrossCompactionWriter.java index ae5bc2e75911..611eceed2f29 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastCrossCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastCrossCompactionWriter.java @@ -85,6 +85,7 @@ public boolean flushNonAlignedChunk(Chunk chunk, ChunkMetadata chunkMetadata, in isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = chunkMetadata.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -123,6 +124,7 @@ public boolean flushAlignedChunk(ChunkMetadataElement chunkMetadataElement, int isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = timeChunkMetadata.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -159,6 +161,7 @@ public boolean flushBatchedValueChunk( isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = timeChunkMetadata.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -200,6 +203,7 @@ public boolean flushAlignedPage(AlignedPageElement alignedPageElement, int subTa isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = timePageHeader.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -239,6 +243,7 @@ public boolean flushBatchedValuePage( isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = timePageHeader.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -270,6 +275,7 @@ public boolean flushNonAlignedPage( isDeviceExistedInTargetFiles[fileIndex] = true; isEmptyFile[fileIndex] = false; lastTime[subTaskId] = pageHeader.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastInnerCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastInnerCompactionWriter.java index e9709af3dbf6..6519101781cc 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastInnerCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/FastInnerCompactionWriter.java @@ -75,6 +75,7 @@ public boolean flushNonAlignedChunk(Chunk chunk, ChunkMetadata chunkMetadata, in flushNonAlignedChunkToFileWriter(fileWriter, chunk, chunkMetadata, subTaskId); lastTime[subTaskId] = chunkMetadata.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -112,6 +113,7 @@ public boolean flushAlignedChunk(ChunkMetadataElement chunkMetadataElement, int fileWriter, timeChunk, timeChunkMetadata, valueChunks, valueChunkMetadatas, subTaskId); lastTime[subTaskId] = timeChunkMetadata.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -142,6 +144,7 @@ public boolean flushBatchedValueChunk( fileWriter, null, timeChunkMetadata, valueChunks, valueChunkMetadatas, subTaskId); lastTime[subTaskId] = timeChunkMetadata.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -180,6 +183,7 @@ public boolean flushAlignedPage(AlignedPageElement alignedPageElement, int subTa subTaskId); lastTime[subTaskId] = timePageHeader.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -215,6 +219,7 @@ public boolean flushBatchedValuePage( subTaskId); lastTime[subTaskId] = timePageHeader.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } @@ -240,6 +245,7 @@ public boolean flushNonAlignedPage( (ChunkWriterImpl) chunkWriters[subTaskId], compressedPageData, pageHeader, subTaskId); lastTime[subTaskId] = pageHeader.getEndTime(); + lastTimeSet[subTaskId] = true; return true; } diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/ReadPointCrossCompactionWriter.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/ReadPointCrossCompactionWriter.java index 84ae142f2815..b92e3e66ff26 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/ReadPointCrossCompactionWriter.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/execute/utils/writer/ReadPointCrossCompactionWriter.java @@ -68,6 +68,7 @@ public void write(TsBlock tsBlock, int subTaskId) throws IOException { isDeviceExistedInTargetFiles[seqFileIndexArray[subTaskId]] = true; isEmptyFile[seqFileIndexArray[subTaskId]] = false; lastTime[subTaskId] = timestamps.getEndTime(); + lastTimeSet[subTaskId] = true; } @Override diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java index aca2d76496b5..b19b54c4f740 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/repair/RepairDataFileScanUtil.java @@ -74,6 +74,7 @@ public class RepairDataFileScanUtil { private ArrayDeviceTimeIndex timeIndex; private boolean hasUnsortedDataOrWrongStatistics; private boolean isBrokenFile; + private boolean previousTimeSet; private long previousTime; private boolean printLog; @@ -84,7 +85,7 @@ public RepairDataFileScanUtil(TsFileResource resource) { public RepairDataFileScanUtil(TsFileResource resource, boolean printLog) { this.resource = resource; this.hasUnsortedDataOrWrongStatistics = false; - this.previousTime = Long.MIN_VALUE; + this.previousTimeSet = false; this.printLog = printLog; } @@ -217,7 +218,7 @@ private void checkAlignedDeviceSeries( } // reset previousTime - previousTime = Long.MIN_VALUE; + previousTimeSet = false; // check timeseries time range if (actualTimeseriesStartTime > actualTimeseriesEndTime) { @@ -290,7 +291,7 @@ private void checkNonAlignedDeviceSeries( actualDeviceEndTime = Math.max(actualDeviceEndTime, timeseriesMetadata.getStatistics().getEndTime()); checkSingleNonAlignedSeries(reader, device, timeseriesMetadata); - previousTime = Long.MIN_VALUE; + previousTimeSet = false; } if (!checkTsFileResource || actualDeviceStartTime > actualDeviceEndTime) { @@ -404,18 +405,20 @@ private void validateTimeData( } private void checkPreviousTimeAndUpdate(IDeviceID deviceID, String measurementId, long time) { - if (previousTime >= time) { + if (previousTimeSet && previousTime >= time) { throw new CompactionLastTimeCheckFailedException( deviceID.toString() + TsFileConstant.PATH_SEPARATOR + measurementId, time, previousTime); } previousTime = time; + previousTimeSet = true; } private void checkPreviousTimeAndUpdate(IDeviceID deviceID, long time) { - if (previousTime >= time) { + if (previousTimeSet && previousTime >= time) { throw new CompactionLastTimeCheckFailedException(deviceID.toString(), time, previousTime); } previousTime = time; + previousTimeSet = true; } public boolean hasUnsortedDataOrWrongStatistics() { From 7916e793c67875894ab8d25ae80bb89c6cbceb18 Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Tue, 24 Dec 2024 18:18:15 +0800 Subject: [PATCH 10/12] remove redundant check --- .../selector/estimator/AbstractCompactionEstimator.java | 5 +---- 1 file changed, 1 insertion(+), 4 deletions(-) diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java index c4257fd715aa..3f46b7f79374 100644 --- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java +++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/storageengine/dataregion/compaction/selector/estimator/AbstractCompactionEstimator.java @@ -142,10 +142,7 @@ protected int calculatingMaxOverlapFileNumInSubCompactionTask(List Date: Thu, 2 Jan 2025 15:39:30 +0800 Subject: [PATCH 11/12] Fix column category names --- .../iotdb/relational/it/session/IoTDBSessionRelationalIT.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java b/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java index b9c4f33fc50f..baeb17c84ce8 100644 --- a/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java @@ -1540,7 +1540,7 @@ public void loadMinMaxTimeAlignedTest() "load_min_max", Arrays.asList("id1", "s1"), Arrays.asList(TSDataType.STRING, TSDataType.INT32), - Arrays.asList(ColumnCategory.ID, ColumnCategory.MEASUREMENT)); + Arrays.asList(ColumnCategory.TAG, ColumnCategory.FIELD)); try (ITsFileWriter writer = new TsFileWriterBuilder().file(file).tableSchema(tableSchema).build()) { From 58f625eff27840a98b74b1d4a26427420aa5e5ff Mon Sep 17 00:00:00 2001 From: Tian Jiang Date: Fri, 3 Jan 2025 09:42:19 +0800 Subject: [PATCH 12/12] fix category --- .../iotdb/relational/it/session/IoTDBSessionRelationalIT.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java b/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java index baeb17c84ce8..9367a4b51baf 100644 --- a/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java +++ b/integration-test/src/test/java/org/apache/iotdb/relational/it/session/IoTDBSessionRelationalIT.java @@ -1486,7 +1486,7 @@ public void insertMinMaxTimeTest() throws IoTDBConnectionException, StatementExe } } session.executeNonQueryStatement("USE db1"); - session.executeNonQueryStatement("CREATE TABLE test_insert_min_max (id1 ID, s1 INT32)"); + session.executeNonQueryStatement("CREATE TABLE test_insert_min_max (id1 TAG, s1 INT32)"); session.executeNonQueryStatement( String.format(