You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@iotdb.apache.org by qi...@apache.org on 2021/07/29 11:36:36 UTC
[iotdb] branch master updated: [IOTDB-1498] MNode Abstraction and
Structure Improvement (#3589)
This is an automated email from the ASF dual-hosted git repository.
qiaojialin pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/iotdb.git
The following commit(s) were added to refs/heads/master by this push:
new 05a8f7c [IOTDB-1498] MNode Abstraction and Structure Improvement (#3589)
05a8f7c is described below
commit 05a8f7cf9faf2080694a171e489184c8ef501e37
Author: zyk990424 <38...@users.noreply.github.com>
AuthorDate: Thu Jul 29 19:35:38 2021 +0800
[IOTDB-1498] MNode Abstraction and Structure Improvement (#3589)
---
.../log/manage/PartitionedSnapshotLogManager.java | 4 +-
.../apache/iotdb/cluster/metadata/CMManager.java | 17 +-
.../iotdb/cluster/partition/MManagerWhiteBox.java | 2 +-
docs/SystemDesign/SchemaManager/SchemaManager.md | 2 +-
.../zh/SystemDesign/SchemaManager/SchemaManager.md | 2 +-
.../iotdb/db/engine/merge/task/MergeTask.java | 6 +-
.../engine/storagegroup/StorageGroupProcessor.java | 10 +-
.../org/apache/iotdb/db/metadata/MManager.java | 832 +++++----------------
.../java/org/apache/iotdb/db/metadata/MTree.java | 464 +++++++-----
.../org/apache/iotdb/db/metadata/MetaUtils.java | 18 +-
.../iotdb/db/metadata/logfile/MLogWriter.java | 4 +-
.../org/apache/iotdb/db/metadata/mnode/IMNode.java | 86 +++
.../mnode/{MNode.java => InternalMNode.java} | 186 ++---
.../org/apache/iotdb/db/metadata/mnode/MNode.java | 331 +-------
.../iotdb/db/metadata/mnode/MeasurementMNode.java | 109 ++-
.../iotdb/db/metadata/mnode/StorageGroupMNode.java | 4 +-
.../apache/iotdb/db/metadata/tag/TagManager.java | 555 ++++++++++++++
.../iotdb/db/metadata/template/Template.java | 4 +
.../db/metadata/template/TemplateManager.java | 141 ++++
.../apache/iotdb/db/qp/executor/PlanExecutor.java | 6 +-
.../db/query/dataset/AlignByDeviceDataSet.java | 6 +-
.../iotdb/db/integration/IoTDBAddSubDeviceIT.java | 11 +-
.../db/integration/IoTDBAutoCreateSchemaIT.java | 6 +-
.../db/integration/IoTDBCreateTimeseriesIT.java | 34 +-
.../apache/iotdb/db/integration/IoTDBLastIT.java | 8 +-
.../iotdb/db/integration/IoTDBMetadataFetchIT.java | 20 +-
.../iotdb/db/metadata/MManagerAdvancedTest.java | 6 +-
.../iotdb/db/metadata/MManagerBasicTest.java | 84 ++-
.../iotdb/db/metadata/MManagerImproveTest.java | 4 +-
.../org/apache/iotdb/db/metadata/MTreeTest.java | 103 +--
.../apache/iotdb/db/metadata/MetaUtilsTest.java | 14 +-
.../apache/iotdb/db/metadata/mnode/MNodeTest.java | 40 +-
.../tsfile/write/schema/IMeasurementSchema.java | 5 +
.../tsfile/write/schema/MeasurementSchema.java | 12 +-
.../write/schema/VectorMeasurementSchema.java | 12 +-
.../write/writer/VectorMeasurementSchemaStub.java | 10 +
36 files changed, 1676 insertions(+), 1482 deletions(-)
diff --git a/cluster/src/main/java/org/apache/iotdb/cluster/log/manage/PartitionedSnapshotLogManager.java b/cluster/src/main/java/org/apache/iotdb/cluster/log/manage/PartitionedSnapshotLogManager.java
index d2fd280..037b977 100644
--- a/cluster/src/main/java/org/apache/iotdb/cluster/log/manage/PartitionedSnapshotLogManager.java
+++ b/cluster/src/main/java/org/apache/iotdb/cluster/log/manage/PartitionedSnapshotLogManager.java
@@ -29,7 +29,7 @@ import org.apache.iotdb.cluster.partition.PartitionTable;
import org.apache.iotdb.cluster.partition.slot.SlotPartitionTable;
import org.apache.iotdb.cluster.rpc.thrift.Node;
import org.apache.iotdb.cluster.server.member.DataGroupMember;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.StorageGroupMNode;
import org.apache.iotdb.db.service.IoTDB;
import org.apache.iotdb.tsfile.write.schema.TimeseriesSchema;
@@ -97,7 +97,7 @@ public abstract class PartitionedSnapshotLogManager<T extends Snapshot> extends
void collectTimeseriesSchemas(List<Integer> requiredSlots) {
slotTimeseries.clear();
List<StorageGroupMNode> allSgNodes = IoTDB.metaManager.getAllStorageGroupNodes();
- for (MNode sgNode : allSgNodes) {
+ for (IMNode sgNode : allSgNodes) {
String storageGroupName = sgNode.getFullPath();
int slot =
SlotPartitionTable.getSlotStrategy()
diff --git a/cluster/src/main/java/org/apache/iotdb/cluster/metadata/CMManager.java b/cluster/src/main/java/org/apache/iotdb/cluster/metadata/CMManager.java
index 160ca67..0a115ba 100644
--- a/cluster/src/main/java/org/apache/iotdb/cluster/metadata/CMManager.java
+++ b/cluster/src/main/java/org/apache/iotdb/cluster/metadata/CMManager.java
@@ -44,7 +44,8 @@ import org.apache.iotdb.db.metadata.MManager;
import org.apache.iotdb.db.metadata.MetaUtils;
import org.apache.iotdb.db.metadata.PartialPath;
import org.apache.iotdb.db.metadata.VectorPartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
+import org.apache.iotdb.db.metadata.mnode.InternalMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.metadata.template.Template;
import org.apache.iotdb.db.qp.constant.SQLConstant;
@@ -247,8 +248,8 @@ public class CMManager extends MManager {
@Override
public Pair<List<PartialPath>, Map<String, Integer>> getSeriesSchemas(List<PartialPath> fullPaths)
throws MetadataException {
- Map<MNode, PartialPath> nodeToPartialPath = new LinkedHashMap<>();
- Map<MNode, List<Integer>> nodeToIndex = new LinkedHashMap<>();
+ Map<IMNode, PartialPath> nodeToPartialPath = new LinkedHashMap<>();
+ Map<IMNode, List<Integer>> nodeToIndex = new LinkedHashMap<>();
for (int i = 0; i < fullPaths.size(); i++) {
PartialPath path = fullPaths.get(i);
MeasurementMNode node = getMeasurementMNode(path);
@@ -432,14 +433,14 @@ public class CMManager extends MManager {
}
@Override
- public MNode getSeriesSchemasAndReadLockDevice(InsertPlan plan)
+ public IMNode getSeriesSchemasAndReadLockDevice(InsertPlan plan)
throws MetadataException, IOException {
MeasurementMNode[] measurementMNodes = new MeasurementMNode[plan.getMeasurements().length];
int nonExistSchemaIndex =
getMNodesLocally(plan.getPrefixPath(), plan.getMeasurements(), measurementMNodes);
if (nonExistSchemaIndex == -1) {
plan.setMeasurementMNodes(measurementMNodes);
- return new MNode(null, plan.getPrefixPath().getDevice());
+ return new InternalMNode(null, plan.getPrefixPath().getDevice());
}
// auto-create schema in IoTDBConfig is always disabled in the cluster version, and we have
// another config in ClusterConfig to do this
@@ -507,7 +508,7 @@ public class CMManager extends MManager {
}
@Override
- public Pair<MNode, Template> getDeviceNodeWithAutoCreate(PartialPath path)
+ public Pair<IMNode, Template> getDeviceNodeWithAutoCreate(PartialPath path)
throws MetadataException, IOException {
return getDeviceNodeWithAutoCreate(
path,
@@ -1382,8 +1383,8 @@ public class CMManager extends MManager {
}
@Override
- public MNode getMNode(MNode deviceMNode, String measurementName) {
- MNode child = deviceMNode.getChild(measurementName);
+ public IMNode getMNode(IMNode deviceMNode, String measurementName) {
+ IMNode child = deviceMNode.getChild(measurementName);
if (child == null) {
child = mRemoteMetaCache.get(deviceMNode.getPartialPath().concatNode(measurementName));
}
diff --git a/cluster/src/test/java/org/apache/iotdb/cluster/partition/MManagerWhiteBox.java b/cluster/src/test/java/org/apache/iotdb/cluster/partition/MManagerWhiteBox.java
index b7e715e..3f1f638 100644
--- a/cluster/src/test/java/org/apache/iotdb/cluster/partition/MManagerWhiteBox.java
+++ b/cluster/src/test/java/org/apache/iotdb/cluster/partition/MManagerWhiteBox.java
@@ -35,7 +35,7 @@ public class MManagerWhiteBox {
MManager manager = constructor.newInstance();
new File(logFilePath).getParentFile().mkdirs();
Whitebox.setInternalState(manager, "logFilePath", logFilePath);
- manager.init();
+ manager.initForMultiMManagerTest();
return manager;
} catch (InstantiationException | IllegalAccessException | InvocationTargetException e) {
e.printStackTrace();
diff --git a/docs/SystemDesign/SchemaManager/SchemaManager.md b/docs/SystemDesign/SchemaManager/SchemaManager.md
index cbfc1cc..ba4568e 100644
--- a/docs/SystemDesign/SchemaManager/SchemaManager.md
+++ b/docs/SystemDesign/SchemaManager/SchemaManager.md
@@ -295,7 +295,7 @@ Schema operation examples and the corresponding parsed mlog record:
> format: 61,path
## TLog
-* org.apache.iotdb.db.metadata.TagLogFile
+* org.apache.iotdb.db.metadata.logfile.TagLogFile
All timeseries tag/attribute information will be saved in the tag file, which defaults to data/system/schema/mlog.bin.
diff --git a/docs/zh/SystemDesign/SchemaManager/SchemaManager.md b/docs/zh/SystemDesign/SchemaManager/SchemaManager.md
index 0493767..a6a7529 100644
--- a/docs/zh/SystemDesign/SchemaManager/SchemaManager.md
+++ b/docs/zh/SystemDesign/SchemaManager/SchemaManager.md
@@ -288,7 +288,7 @@ mlog.bin 存储二进制编码。我们可以使用 [MlogParser Tool](https://io
> 格式:61,path
## 标签文件
-* org.apache.iotdb.db.metadata.TagLogFile
+* org.apache.iotdb.db.metadata.logfile.TagLogFile
所有时间序列的标签/属性信息都会保存在标签文件中,此文件默认为 data/system/schema/mlog.bin。
diff --git a/server/src/main/java/org/apache/iotdb/db/engine/merge/task/MergeTask.java b/server/src/main/java/org/apache/iotdb/db/engine/merge/task/MergeTask.java
index 381651c..d35e75b 100644
--- a/server/src/main/java/org/apache/iotdb/db/engine/merge/task/MergeTask.java
+++ b/server/src/main/java/org/apache/iotdb/db/engine/merge/task/MergeTask.java
@@ -25,7 +25,7 @@ import org.apache.iotdb.db.engine.merge.recover.MergeLogger;
import org.apache.iotdb.db.engine.storagegroup.TsFileResource;
import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.service.IoTDB;
import org.apache.iotdb.db.utils.MergeUtils;
@@ -149,9 +149,9 @@ public class MergeTask implements Callable<Void> {
Map<PartialPath, IMeasurementSchema> measurementSchemaMap = new HashMap<>();
List<PartialPath> unmergedSeries = new ArrayList<>();
for (PartialPath device : devices) {
- MNode deviceNode = IoTDB.metaManager.getNodeByPath(device);
+ IMNode deviceNode = IoTDB.metaManager.getNodeByPath(device);
// todo add template merge logic
- for (Entry<String, MNode> entry : deviceNode.getChildren().entrySet()) {
+ for (Entry<String, IMNode> entry : deviceNode.getChildren().entrySet()) {
PartialPath path = device.concatNode(entry.getKey());
measurementSchemaMap.put(path, ((MeasurementMNode) entry.getValue()).getSchema());
unmergedSeries.add(path);
diff --git a/server/src/main/java/org/apache/iotdb/db/engine/storagegroup/StorageGroupProcessor.java b/server/src/main/java/org/apache/iotdb/db/engine/storagegroup/StorageGroupProcessor.java
index e69982b..0481b27 100755
--- a/server/src/main/java/org/apache/iotdb/db/engine/storagegroup/StorageGroupProcessor.java
+++ b/server/src/main/java/org/apache/iotdb/db/engine/storagegroup/StorageGroupProcessor.java
@@ -55,7 +55,7 @@ import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.exception.query.OutOfTTLException;
import org.apache.iotdb.db.exception.query.QueryProcessException;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.qp.physical.crud.DeletePlan;
import org.apache.iotdb.db.qp.physical.crud.InsertRowPlan;
@@ -1933,9 +1933,9 @@ public class StorageGroupProcessor {
return;
}
try {
- MNode node = IoTDB.metaManager.getDeviceNode(deviceId);
+ IMNode node = IoTDB.metaManager.getDeviceNode(deviceId);
- for (MNode measurementNode : node.getChildren().values()) {
+ for (IMNode measurementNode : node.getChildren().values()) {
if (measurementNode != null
&& originalPath.matchFullPath(measurementNode.getPartialPath())) {
TimeValuePair lastPair = ((MeasurementMNode) measurementNode).getCachedLast();
@@ -2278,9 +2278,9 @@ public class StorageGroupProcessor {
return;
}
try {
- MNode node = IoTDB.metaManager.getDeviceNode(deviceId);
+ IMNode node = IoTDB.metaManager.getDeviceNode(deviceId);
- for (MNode measurementNode : node.getChildren().values()) {
+ for (IMNode measurementNode : node.getChildren().values()) {
if (measurementNode != null) {
((MeasurementMNode) measurementNode).resetCache();
logger.debug(
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/MManager.java b/server/src/main/java/org/apache/iotdb/db/metadata/MManager.java
index b5e3ff0..9c686f5 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/MManager.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/MManager.java
@@ -22,28 +22,25 @@ import org.apache.iotdb.db.conf.IoTDBConfig;
import org.apache.iotdb.db.conf.IoTDBDescriptor;
import org.apache.iotdb.db.engine.StorageEngine;
import org.apache.iotdb.db.engine.fileSystem.SystemFileFactory;
-import org.apache.iotdb.db.engine.storagegroup.StorageGroupProcessor;
import org.apache.iotdb.db.engine.trigger.executor.TriggerEngine;
-import org.apache.iotdb.db.exception.StorageEngineException;
import org.apache.iotdb.db.exception.metadata.AliasAlreadyExistException;
import org.apache.iotdb.db.exception.metadata.AlignedTimeseriesException;
import org.apache.iotdb.db.exception.metadata.DataTypeMismatchException;
import org.apache.iotdb.db.exception.metadata.DeleteFailedException;
-import org.apache.iotdb.db.exception.metadata.DuplicatedTemplateException;
import org.apache.iotdb.db.exception.metadata.IllegalPathException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.exception.metadata.PathAlreadyExistException;
import org.apache.iotdb.db.exception.metadata.PathNotExistException;
import org.apache.iotdb.db.exception.metadata.StorageGroupAlreadySetException;
import org.apache.iotdb.db.exception.metadata.StorageGroupNotSetException;
-import org.apache.iotdb.db.exception.metadata.UndefinedTemplateException;
import org.apache.iotdb.db.metadata.logfile.MLogReader;
import org.apache.iotdb.db.metadata.logfile.MLogWriter;
-import org.apache.iotdb.db.metadata.logfile.TagLogFile;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.metadata.mnode.StorageGroupMNode;
+import org.apache.iotdb.db.metadata.tag.TagManager;
import org.apache.iotdb.db.metadata.template.Template;
+import org.apache.iotdb.db.metadata.template.TemplateManager;
import org.apache.iotdb.db.monitor.MonitorConstants;
import org.apache.iotdb.db.qp.constant.SQLConstant;
import org.apache.iotdb.db.qp.physical.PhysicalPlan;
@@ -91,13 +88,11 @@ import org.slf4j.LoggerFactory;
import java.io.File;
import java.io.IOException;
-import java.nio.file.Files;
import java.util.ArrayDeque;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
-import java.util.Comparator;
import java.util.Deque;
import java.util.HashMap;
import java.util.HashSet;
@@ -107,14 +102,11 @@ import java.util.List;
import java.util.Map;
import java.util.Map.Entry;
import java.util.Set;
-import java.util.concurrent.ConcurrentHashMap;
-import java.util.concurrent.CopyOnWriteArraySet;
import java.util.concurrent.Executors;
import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicLong;
-import static java.util.stream.Collectors.toList;
import static org.apache.iotdb.db.utils.EncodingInferenceUtils.getDefaultEncoding;
import static org.apache.iotdb.tsfile.common.constant.TsFileConstant.PATH_SEPARATOR;
@@ -126,51 +118,40 @@ import static org.apache.iotdb.tsfile.common.constant.TsFileConstant.PATH_SEPARA
@SuppressWarnings("java:S1135") // ignore todos
public class MManager {
- public static final String TIME_SERIES_TREE_HEADER = "=== Timeseries Tree ===\n\n";
- private static final String TAG_FORMAT = "tag key is %s, tag value is %s, tlog offset is %d";
- private static final String DEBUG_MSG = "%s : TimeSeries %s is removed from tag inverted index, ";
- private static final String DEBUG_MSG_1 =
- "%s: TimeSeries %s's tag info has been removed from tag inverted index ";
- private static final String PREVIOUS_CONDITION =
- "before deleting it, tag key is %s, tag value is %s, tlog offset is %d, contains key %b";
-
private static final Logger logger = LoggerFactory.getLogger(MManager.class);
+ public static final String TIME_SERIES_TREE_HEADER = "=== Timeseries Tree ===\n\n";
+
/** A thread will check whether the MTree is modified lately each such interval. Unit: second */
private static final long MTREE_SNAPSHOT_THREAD_CHECK_TIME = 600L;
- private final int mtreeSnapshotInterval;
- private final long mtreeSnapshotThresholdTime;
- // the log file seriesPath
- private String logFilePath;
- private String mtreeSnapshotPath;
- private String mtreeSnapshotTmpPath;
- private MTree mtree;
- private MLogWriter logWriter;
- private TagLogFile tagLogFile;
+ protected static IoTDBConfig config = IoTDBDescriptor.getInstance().getConfig();
+ /** threshold total size of MTree */
+ private static final long MTREE_SIZE_THRESHOLD = config.getAllocateMemoryForSchema();
+
+ private static final int ESTIMATED_SERIES_SIZE = config.getEstimatedSeriesSize();
+
private boolean isRecovering;
- // device -> DeviceMNode
- private RandomDeleteCache<PartialPath, Pair<MNode, Template>> mNodeCache;
- // tag key -> tag value -> LeafMNode
- private Map<String, Map<String, Set<MeasurementMNode>>> tagIndex = new ConcurrentHashMap<>();
+ private boolean initialized;
+ private boolean allowToCreateNewSeries = true;
private AtomicLong totalSeriesNumber = new AtomicLong();
- private boolean initialized;
- protected static IoTDBConfig config = IoTDBDescriptor.getInstance().getConfig();
- private File logFile;
+ private final int mtreeSnapshotInterval;
+ private final long mtreeSnapshotThresholdTime;
private ScheduledExecutorService timedCreateMTreeSnapshotThread;
private ScheduledExecutorService timedForceMLogThread;
- /** threshold total size of MTree */
- private static final long MTREE_SIZE_THRESHOLD = config.getAllocateMemoryForSchema();
-
- private boolean allowToCreateNewSeries = true;
-
- private static final int ESTIMATED_SERIES_SIZE = config.getEstimatedSeriesSize();
+ // the log file seriesPath
+ private String logFilePath;
+ private File logFile;
+ private MLogWriter logWriter;
- // template name -> template
- private Map<String, Template> templateMap = new ConcurrentHashMap<>();
+ private MTree mtree;
+ // device -> DeviceMNode
+ private RandomDeleteCache<PartialPath, Pair<IMNode, Template>> mNodeCache;
+ private TagManager tagManager = TagManager.getInstance();
+ private TemplateManager templateManager = TemplateManager.getInstance();
private static class MManagerHolder {
@@ -194,18 +175,16 @@ public class MManager {
}
}
logFilePath = schemaDir + File.separator + MetadataConstant.METADATA_LOG;
- mtreeSnapshotPath = schemaDir + File.separator + MetadataConstant.MTREE_SNAPSHOT;
- mtreeSnapshotTmpPath = schemaDir + File.separator + MetadataConstant.MTREE_SNAPSHOT_TMP;
// do not write log when recover
isRecovering = true;
int cacheSize = config.getmManagerCacheSize();
mNodeCache =
- new RandomDeleteCache<PartialPath, Pair<MNode, Template>>(cacheSize) {
+ new RandomDeleteCache<PartialPath, Pair<IMNode, Template>>(cacheSize) {
@Override
- public Pair<MNode, Template> loadObjectByKey(PartialPath key) throws CacheException {
+ public Pair<IMNode, Template> loadObjectByKey(PartialPath key) throws CacheException {
try {
return mtree.getNodeByPathWithStorageGroupCheck(key);
} catch (MetadataException e) {
@@ -240,9 +219,12 @@ public class MManager {
logFile = SystemFileFactory.INSTANCE.getFile(logFilePath);
try {
- tagLogFile = new TagLogFile(config.getSchemaDir(), MetadataConstant.TAG_LOG);
-
isRecovering = true;
+
+ tagManager.init();
+ mtree = new MTree();
+ mtree.init();
+
int lineNumber = initFromLog(logFile);
logWriter = new MLogWriter(config.getSchemaDir(), MetadataConstant.METADATA_LOG);
@@ -255,26 +237,22 @@ public class MManager {
initialized = true;
}
+ /**
+ * Attention!!!!!, this method could only be used for Tests involving multiple mmanagers. The
+ * singleton of templateManager and tagManager will cause interference between mmanagers if one of
+ * the mmanagers invoke init method or clear method
+ */
+ @TestOnly
+ public void initForMultiMManagerTest() {
+ templateManager = TemplateManager.getNewInstanceForTest();
+ tagManager = TagManager.getNewInstanceForTest();
+ init();
+ }
+
/** @return line number of the logFile */
@SuppressWarnings("squid:S3776")
private int initFromLog(File logFile) throws IOException {
- File tmpFile = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath);
- if (tmpFile.exists()) {
- logger.warn("Creating MTree snapshot not successful before crashing...");
- Files.delete(tmpFile.toPath());
- }
-
- File mtreeSnapshot = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotPath);
long time = System.currentTimeMillis();
- if (!mtreeSnapshot.exists()) {
- mtree = new MTree();
- } else {
- mtree = MTree.deserializeFrom(mtreeSnapshot);
- logger.debug(
- "spend {} ms to deserialize mtree from snapshot", System.currentTimeMillis() - time);
- }
-
- time = System.currentTimeMillis();
// init the metadata from the operation log
if (logFile.exists()) {
int idx = 0;
@@ -316,23 +294,46 @@ public class MManager {
return idx;
}
+ private void checkMTreeModified() {
+ if (logWriter == null || logFile == null) {
+ // the logWriter is not initialized now, we skip the check once.
+ return;
+ }
+ if (System.currentTimeMillis() - logFile.lastModified() >= mtreeSnapshotThresholdTime
+ || logWriter.getLogNum() >= mtreeSnapshotInterval) {
+ logger.info(
+ "New mlog line number: {}, time from last modification: {} ms",
+ logWriter.getLogNum(),
+ System.currentTimeMillis() - logFile.lastModified());
+ createMTreeSnapshot();
+ }
+ }
+
+ public void createMTreeSnapshot() {
+ try {
+ mtree.createSnapshot();
+ logWriter.clear();
+ } catch (IOException e) {
+ logger.warn("Failed to create MTree snapshot", e);
+ }
+ }
+
/** function for clearing MTree */
- public void clear() {
+ public synchronized void clear() {
try {
- templateMap.clear();
- this.mtree = new MTree();
- this.mNodeCache.clear();
- this.tagIndex.clear();
+ if (this.mtree != null) {
+ this.mtree.clear();
+ }
+ if (this.mNodeCache != null) {
+ this.mNodeCache.clear();
+ }
this.totalSeriesNumber.set(0);
- this.templateMap.clear();
+ this.templateManager.clear();
if (logWriter != null) {
logWriter.close();
logWriter = null;
}
- if (tagLogFile != null) {
- tagLogFile.close();
- tagLogFile = null;
- }
+ tagManager.clear();
initialized = false;
if (config.isEnableMTreeSnapshot() && timedCreateMTreeSnapshotThread != null) {
timedCreateMTreeSnapshotThread.shutdownNow();
@@ -460,10 +461,7 @@ public class MManager {
if (entry.getKey() == null || entry.getValue() == null) {
continue;
}
- tagIndex
- .computeIfAbsent(entry.getKey(), k -> new ConcurrentHashMap<>())
- .computeIfAbsent(entry.getValue(), v -> new CopyOnWriteArraySet<>())
- .add(leafMNode);
+ tagManager.addIndex(entry.getKey(), entry.getValue(), leafMNode);
}
}
@@ -479,7 +477,7 @@ public class MManager {
// either tags or attributes is not empty
if ((plan.getTags() != null && !plan.getTags().isEmpty())
|| (plan.getAttributes() != null && !plan.getAttributes().isEmpty())) {
- offset = tagLogFile.write(plan.getTags(), plan.getAttributes());
+ offset = tagManager.writeTagFile(plan.getTags(), plan.getAttributes());
}
plan.setTagOffset(offset);
logWriter.createTimeseries(plan);
@@ -591,7 +589,7 @@ public class MManager {
// for not support deleting part of aligned timeseies
// should be removed after partial deletion is supported
- MNode lastNode = getNodeByPath(allTimeseries.get(0));
+ IMNode lastNode = getNodeByPath(allTimeseries.get(0));
if (lastNode instanceof MeasurementMNode) {
IMeasurementSchema schema = ((MeasurementMNode) lastNode).getSchema();
if (schema instanceof VectorMeasurementSchema) {
@@ -639,43 +637,7 @@ public class MManager {
/** remove the node from the tag inverted index */
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
private void removeFromTagInvertedIndex(MeasurementMNode node) throws IOException {
- if (node.getOffset() < 0) {
- return;
- }
- Map<String, String> tagMap =
- tagLogFile.readTag(config.getTagAttributeTotalSize(), node.getOffset());
- if (tagMap != null) {
- for (Entry<String, String> entry : tagMap.entrySet()) {
- if (tagIndex.containsKey(entry.getKey())
- && tagIndex.get(entry.getKey()).containsKey(entry.getValue())) {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG, "Delete" + TAG_FORMAT, node.getFullPath()),
- entry.getKey(),
- entry.getValue(),
- node.getOffset()));
- }
- tagIndex.get(entry.getKey()).get(entry.getValue()).remove(node);
- if (tagIndex.get(entry.getKey()).get(entry.getValue()).isEmpty()) {
- tagIndex.get(entry.getKey()).remove(entry.getValue());
- if (tagIndex.get(entry.getKey()).isEmpty()) {
- tagIndex.remove(entry.getKey());
- }
- }
- } else {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG_1, "Delete" + PREVIOUS_CONDITION, node.getFullPath()),
- entry.getKey(),
- entry.getValue(),
- node.getOffset(),
- tagIndex.containsKey(entry.getKey())));
- }
- }
- }
- }
+ tagManager.removeFromTagInvertedIndex(node);
}
/**
@@ -935,70 +897,21 @@ public class MManager {
return mtree.getNodesCountInGivenLevel(prefixPath, level);
}
+ public List<ShowTimeSeriesResult> showTimeseries(ShowTimeSeriesPlan plan, QueryContext context)
+ throws MetadataException {
+ // show timeseries with index
+ if (plan.getKey() != null && plan.getValue() != null) {
+ return showTimeseriesWithIndex(plan, context);
+ } else {
+ return showTimeseriesWithoutIndex(plan, context);
+ }
+ }
+
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
private List<ShowTimeSeriesResult> showTimeseriesWithIndex(
ShowTimeSeriesPlan plan, QueryContext context) throws MetadataException {
- if (!tagIndex.containsKey(plan.getKey())) {
- throw new MetadataException("The key " + plan.getKey() + " is not a tag.", true);
- }
- Map<String, Set<MeasurementMNode>> value2Node = tagIndex.get(plan.getKey());
- if (value2Node.isEmpty()) {
- throw new MetadataException("The key " + plan.getKey() + " is not a tag.");
- }
- List<MeasurementMNode> allMatchedNodes = new ArrayList<>();
- if (plan.isContains()) {
- for (Entry<String, Set<MeasurementMNode>> entry : value2Node.entrySet()) {
- if (entry.getKey() == null || entry.getValue() == null) {
- continue;
- }
- String tagValue = entry.getKey();
- if (tagValue.contains(plan.getValue())) {
- allMatchedNodes.addAll(entry.getValue());
- }
- }
- } else {
- for (Entry<String, Set<MeasurementMNode>> entry : value2Node.entrySet()) {
- if (entry.getKey() == null || entry.getValue() == null) {
- continue;
- }
- String tagValue = entry.getKey();
- if (plan.getValue().equals(tagValue)) {
- allMatchedNodes.addAll(entry.getValue());
- }
- }
- }
-
- // if ordered by heat, we sort all the timeseries by the descending order of the last insert
- // timestamp
- if (plan.isOrderByHeat()) {
- List<StorageGroupProcessor> list;
- try {
- list =
- StorageEngine.getInstance()
- .mergeLock(allMatchedNodes.stream().map(MNode::getPartialPath).collect(toList()));
- try {
- allMatchedNodes =
- allMatchedNodes.stream()
- .sorted(
- Comparator.comparingLong(
- (MeasurementMNode mNode) -> MTree.getLastTimeStamp(mNode, context))
- .reversed()
- .thenComparing(MNode::getFullPath))
- .collect(toList());
- } finally {
- StorageEngine.getInstance().mergeUnLock(list);
- }
- } catch (StorageEngineException e) {
- throw new MetadataException(e);
- }
- } else {
- // otherwise, we just sort them by the alphabetical order
- allMatchedNodes =
- allMatchedNodes.stream()
- .sorted(Comparator.comparing(MNode::getFullPath))
- .collect(toList());
- }
+ List<MeasurementMNode> allMatchedNodes = tagManager.getMatchedTimeseriesInIndex(plan, context);
List<ShowTimeSeriesResult> res = new LinkedList<>();
String[] prefixNodes = plan.getPath().getNodes();
@@ -1016,7 +929,7 @@ public class MManager {
}
try {
Pair<Map<String, String>, Map<String, String>> tagAndAttributePair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leaf.getOffset());
+ tagManager.readTagFile(leaf.getOffset());
IMeasurementSchema measurementSchema = leaf.getSchema();
res.add(
new ShowTimeSeriesResult(
@@ -1054,16 +967,6 @@ public class MManager {
return true;
}
- public List<ShowTimeSeriesResult> showTimeseries(ShowTimeSeriesPlan plan, QueryContext context)
- throws MetadataException {
- // show timeseries with index
- if (plan.getKey() != null && plan.getValue() != null) {
- return showTimeseriesWithIndex(plan, context);
- } else {
- return showTimeseriesWithoutIndex(plan, context);
- }
- }
-
/**
* Get the result of ShowTimeseriesPlan
*
@@ -1084,7 +987,7 @@ public class MManager {
Pair<Map<String, String>, Map<String, String>> tagAndAttributePair =
new Pair<>(Collections.emptyMap(), Collections.emptyMap());
if (tagFileOffset >= 0) {
- tagAndAttributePair = tagLogFile.read(config.getTagAttributeTotalSize(), tagFileOffset);
+ tagAndAttributePair = tagManager.readTagFile(tagFileOffset);
}
res.add(
new ShowTimeSeriesResult(
@@ -1114,8 +1017,8 @@ public class MManager {
*/
public IMeasurementSchema getSeriesSchema(PartialPath device, String measurement)
throws MetadataException {
- MNode deviceMNode = getDeviceNode(device);
- MeasurementMNode measurementMNode = (MeasurementMNode) deviceMNode.getChild(measurement);
+ IMNode deviceIMNode = getDeviceNode(device);
+ MeasurementMNode measurementMNode = (MeasurementMNode) deviceIMNode.getChild(measurement);
if (measurementMNode == null) {
// Just for the initial adaptation of the template functionality and merge functionality
// The getSeriesSchema interface needs to be cleaned up later
@@ -1191,8 +1094,8 @@ public class MManager {
*/
public Pair<List<PartialPath>, Map<String, Integer>> getSeriesSchemas(List<PartialPath> fullPaths)
throws MetadataException {
- Map<MNode, PartialPath> nodeToPartialPath = new LinkedHashMap<>();
- Map<MNode, List<Integer>> nodeToIndex = new LinkedHashMap<>();
+ Map<IMNode, PartialPath> nodeToPartialPath = new LinkedHashMap<>();
+ Map<IMNode, List<Integer>> nodeToIndex = new LinkedHashMap<>();
for (int i = 0; i < fullPaths.size(); i++) {
PartialPath path = fullPaths.get(i);
// use dfs to collect paths
@@ -1204,8 +1107,8 @@ public class MManager {
protected void getNodeToPartialPath(
MeasurementMNode node,
- Map<MNode, PartialPath> nodeToPartialPath,
- Map<MNode, List<Integer>> nodeToIndex,
+ Map<IMNode, PartialPath> nodeToPartialPath,
+ Map<IMNode, List<Integer>> nodeToIndex,
PartialPath path,
int index)
throws MetadataException {
@@ -1234,8 +1137,8 @@ public class MManager {
protected Pair<List<PartialPath>, Map<String, Integer>> getPair(
List<PartialPath> fullPaths,
- Map<MNode, PartialPath> nodeToPartialPath,
- Map<MNode, List<Integer>> nodeToIndex)
+ Map<IMNode, PartialPath> nodeToPartialPath,
+ Map<IMNode, List<Integer>> nodeToIndex)
throws MetadataException {
Map<String, Integer> indexMap = new HashMap<>();
int i = 0;
@@ -1291,7 +1194,7 @@ public class MManager {
}
/** Get node by path */
- public MNode getNodeByPath(PartialPath path) throws MetadataException {
+ public IMNode getNodeByPath(PartialPath path) throws MetadataException {
return mtree.getNodeByPath(path);
}
@@ -1320,10 +1223,10 @@ public class MManager {
* needs to make the Meta group aware of the creation of an SG, so an exception needs to be
* thrown here
*/
- public Pair<MNode, Template> getDeviceNodeWithAutoCreate(
+ public Pair<IMNode, Template> getDeviceNodeWithAutoCreate(
PartialPath path, boolean autoCreateSchema, boolean allowCreateSg, int sgLevel)
throws IOException, MetadataException {
- Pair<MNode, Template> node;
+ Pair<IMNode, Template> node;
boolean shouldSetStorageGroup;
try {
node = mNodeCache.get(path);
@@ -1360,7 +1263,7 @@ public class MManager {
}
/** !!!!!!Attention!!!!! must call the return node's readUnlock() if you call this method. */
- public Pair<MNode, Template> getDeviceNodeWithAutoCreate(PartialPath path)
+ public Pair<IMNode, Template> getDeviceNodeWithAutoCreate(PartialPath path)
throws MetadataException, IOException {
return getDeviceNodeWithAutoCreate(
path, config.isAutoCreateSchemaEnabled(), true, config.getDefaultStorageGroupLevel());
@@ -1371,13 +1274,13 @@ public class MManager {
throws PathNotExistException {
Set<IMeasurementSchema> res = new HashSet<>();
try {
- Pair<MNode, Template> mNodeTemplatePair = mNodeCache.get(path);
+ Pair<IMNode, Template> mNodeTemplatePair = mNodeCache.get(path);
if (mNodeTemplatePair.left.getDeviceTemplate() != null) {
mNodeTemplatePair.right = mNodeTemplatePair.left.getDeviceTemplate();
}
- for (MNode mNode : mNodeTemplatePair.left.getChildren().values()) {
- MeasurementMNode measurementMNode = (MeasurementMNode) mNode;
+ for (IMNode IMNode : mNodeTemplatePair.left.getChildren().values()) {
+ MeasurementMNode measurementMNode = (MeasurementMNode) IMNode;
res.add(measurementMNode.getSchema());
}
@@ -1392,8 +1295,8 @@ public class MManager {
return new ArrayList<>(res);
}
- public MNode getDeviceNode(PartialPath path) throws MetadataException {
- MNode node;
+ public IMNode getDeviceNode(PartialPath path) throws MetadataException {
+ IMNode node;
try {
node = mNodeCache.get(path).left;
return node;
@@ -1412,7 +1315,7 @@ public class MManager {
public String getDeviceId(PartialPath path) {
String device = null;
try {
- MNode deviceNode = getDeviceNode(path);
+ IMNode deviceNode = getDeviceNode(path);
device = deviceNode.getFullPath();
} catch (MetadataException | NullPointerException e) {
// Cannot get deviceId from MManager, return the input deviceId
@@ -1487,104 +1390,46 @@ public class MManager {
Map<String, String> attributesMap,
PartialPath fullPath)
throws MetadataException, IOException {
- MNode mNode = mtree.getNodeByPath(fullPath);
- if (!(mNode instanceof MeasurementMNode)) {
+ IMNode IMNode = mtree.getNodeByPath(fullPath);
+ if (!(IMNode instanceof MeasurementMNode)) {
throw new PathNotExistException(fullPath.getFullPath());
}
- MeasurementMNode leafMNode = (MeasurementMNode) mNode;
+ MeasurementMNode leafMNode = (MeasurementMNode) IMNode;
// upsert alias
- if (alias != null && !alias.equals(leafMNode.getAlias())) {
- if (!leafMNode.getParent().addAlias(alias, leafMNode)) {
- throw new MetadataException("The alias already exists.");
- }
-
- if (leafMNode.getAlias() != null) {
- leafMNode.getParent().deleteAliasChild(leafMNode.getAlias());
- }
-
- leafMNode.setAlias(alias);
- // persist to WAL
- logWriter.changeAlias(fullPath, alias);
- }
+ upsertAlias(alias, fullPath, leafMNode);
if (tagsMap == null && attributesMap == null) {
return;
}
// no tag or attribute, we need to add a new record in log
if (leafMNode.getOffset() < 0) {
- long offset = tagLogFile.write(tagsMap, attributesMap);
+ long offset = tagManager.writeTagFile(tagsMap, attributesMap);
logWriter.changeOffset(fullPath, offset);
leafMNode.setOffset(offset);
// update inverted Index map
- if (tagsMap != null) {
- for (Entry<String, String> entry : tagsMap.entrySet()) {
- tagIndex
- .computeIfAbsent(entry.getKey(), k -> new ConcurrentHashMap<>())
- .computeIfAbsent(entry.getValue(), v -> new CopyOnWriteArraySet<>())
- .add(leafMNode);
- }
- }
+ tagManager.addIndex(tagsMap, leafMNode);
return;
}
- Pair<Map<String, String>, Map<String, String>> pair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
-
- if (tagsMap != null) {
- for (Entry<String, String> entry : tagsMap.entrySet()) {
- String key = entry.getKey();
- String value = entry.getValue();
- String beforeValue = pair.left.get(key);
- pair.left.put(key, value);
- // if the key has existed and the value is not equal to the new one
- // we should remove before key-value from inverted index map
- if (beforeValue != null && !beforeValue.equals(value)) {
-
- if (tagIndex.containsKey(key) && tagIndex.get(key).containsKey(beforeValue)) {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG, "Upsert" + TAG_FORMAT, leafMNode.getFullPath()),
- key,
- beforeValue,
- leafMNode.getOffset()));
- }
+ tagManager.updateTagsAndAttributes(tagsMap, attributesMap, leafMNode);
+ }
- tagIndex.get(key).get(beforeValue).remove(leafMNode);
- if (tagIndex.get(key).get(beforeValue).isEmpty()) {
- tagIndex.get(key).remove(beforeValue);
- }
- } else {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(
- DEBUG_MSG_1, "Upsert" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
- key,
- beforeValue,
- leafMNode.getOffset(),
- tagIndex.containsKey(key)));
- }
- }
- }
+ private void upsertAlias(String alias, PartialPath fullPath, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+ // upsert alias
+ if (alias != null && !alias.equals(leafMNode.getAlias())) {
+ if (!leafMNode.getParent().addAlias(alias, leafMNode)) {
+ throw new MetadataException("The alias already exists.");
+ }
- // if the key doesn't exist or the value is not equal to the new one
- // we should add a new key-value to inverted index map
- if (beforeValue == null || !beforeValue.equals(value)) {
- tagIndex
- .computeIfAbsent(key, k -> new ConcurrentHashMap<>())
- .computeIfAbsent(value, v -> new CopyOnWriteArraySet<>())
- .add(leafMNode);
- }
+ if (leafMNode.getAlias() != null) {
+ leafMNode.getParent().deleteAliasChild(leafMNode.getAlias());
}
- }
- if (attributesMap != null) {
- pair.right.putAll(attributesMap);
+ leafMNode.setAlias(alias);
+ // persist to WAL
+ logWriter.changeAlias(fullPath, alias);
}
-
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
}
/**
@@ -1595,34 +1440,20 @@ public class MManager {
*/
public void addAttributes(Map<String, String> attributesMap, PartialPath fullPath)
throws MetadataException, IOException {
- MNode mNode = mtree.getNodeByPath(fullPath);
- if (!(mNode instanceof MeasurementMNode)) {
+ IMNode IMNode = mtree.getNodeByPath(fullPath);
+ if (!(IMNode instanceof MeasurementMNode)) {
throw new PathNotExistException(fullPath.getFullPath());
}
- MeasurementMNode leafMNode = (MeasurementMNode) mNode;
+ MeasurementMNode leafMNode = (MeasurementMNode) IMNode;
// no tag or attribute, we need to add a new record in log
if (leafMNode.getOffset() < 0) {
- long offset = tagLogFile.write(Collections.emptyMap(), attributesMap);
+ long offset = tagManager.writeTagFile(Collections.emptyMap(), attributesMap);
logWriter.changeOffset(fullPath, offset);
leafMNode.setOffset(offset);
return;
}
- Pair<Map<String, String>, Map<String, String>> pair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
-
- for (Entry<String, String> entry : attributesMap.entrySet()) {
- String key = entry.getKey();
- String value = entry.getValue();
- if (pair.right.containsKey(key)) {
- throw new MetadataException(
- String.format("TimeSeries [%s] already has the attribute [%s].", fullPath, key));
- }
- pair.right.put(key, value);
- }
-
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+ tagManager.addAttributes(attributesMap, fullPath, leafMNode);
}
/**
@@ -1633,49 +1464,22 @@ public class MManager {
*/
public void addTags(Map<String, String> tagsMap, PartialPath fullPath)
throws MetadataException, IOException {
- MNode mNode = mtree.getNodeByPath(fullPath);
- if (!(mNode instanceof MeasurementMNode)) {
+ IMNode IMNode = mtree.getNodeByPath(fullPath);
+ if (!(IMNode instanceof MeasurementMNode)) {
throw new PathNotExistException(fullPath.getFullPath());
}
- MeasurementMNode leafMNode = (MeasurementMNode) mNode;
+ MeasurementMNode leafMNode = (MeasurementMNode) IMNode;
// no tag or attribute, we need to add a new record in log
if (leafMNode.getOffset() < 0) {
- long offset = tagLogFile.write(tagsMap, Collections.emptyMap());
+ long offset = tagManager.writeTagFile(tagsMap, Collections.emptyMap());
logWriter.changeOffset(fullPath, offset);
leafMNode.setOffset(offset);
// update inverted Index map
- for (Entry<String, String> entry : tagsMap.entrySet()) {
- tagIndex
- .computeIfAbsent(entry.getKey(), k -> new ConcurrentHashMap<>())
- .computeIfAbsent(entry.getValue(), v -> new CopyOnWriteArraySet<>())
- .add(leafMNode);
- }
+ tagManager.addIndex(tagsMap, leafMNode);
return;
}
- Pair<Map<String, String>, Map<String, String>> pair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
-
- for (Entry<String, String> entry : tagsMap.entrySet()) {
- String key = entry.getKey();
- String value = entry.getValue();
- if (pair.left.containsKey(key)) {
- throw new MetadataException(
- String.format("TimeSeries [%s] already has the tag [%s].", fullPath, key));
- }
- pair.left.put(key, value);
- }
-
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
-
- // update tag inverted map
- tagsMap.forEach(
- (key, value) ->
- tagIndex
- .computeIfAbsent(key, k -> new ConcurrentHashMap<>())
- .computeIfAbsent(value, v -> new CopyOnWriteArraySet<>())
- .add(leafMNode));
+ tagManager.addTags(tagsMap, fullPath, leafMNode);
}
/**
@@ -1687,75 +1491,16 @@ public class MManager {
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
public void dropTagsOrAttributes(Set<String> keySet, PartialPath fullPath)
throws MetadataException, IOException {
- MNode mNode = mtree.getNodeByPath(fullPath);
- if (!(mNode instanceof MeasurementMNode)) {
+ IMNode IMNode = mtree.getNodeByPath(fullPath);
+ if (!(IMNode instanceof MeasurementMNode)) {
throw new PathNotExistException(fullPath.getFullPath());
}
- MeasurementMNode leafMNode = (MeasurementMNode) mNode;
+ MeasurementMNode leafMNode = (MeasurementMNode) IMNode;
// no tag or attribute, just do nothing.
if (leafMNode.getOffset() < 0) {
return;
}
- Pair<Map<String, String>, Map<String, String>> pair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
-
- Map<String, String> deleteTag = new HashMap<>();
- for (String key : keySet) {
- // check tag map
- // check attribute map
- String removeVal = pair.left.remove(key);
- if (removeVal != null) {
- deleteTag.put(key, removeVal);
- } else {
- removeVal = pair.right.remove(key);
- if (removeVal == null) {
- logger.warn("TimeSeries [{}] does not have tag/attribute [{}]", fullPath, key);
- }
- }
- }
-
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
-
- Map<String, Set<MeasurementMNode>> tagVal2LeafMNodeSet;
- Set<MeasurementMNode> MMNodes;
- for (Entry<String, String> entry : deleteTag.entrySet()) {
- String key = entry.getKey();
- String value = entry.getValue();
- // change the tag inverted index map
- tagVal2LeafMNodeSet = tagIndex.get(key);
- if (tagVal2LeafMNodeSet != null) {
- MMNodes = tagVal2LeafMNodeSet.get(value);
- if (MMNodes != null) {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG, "Drop" + TAG_FORMAT, leafMNode.getFullPath()),
- entry.getKey(),
- entry.getValue(),
- leafMNode.getOffset()));
- }
-
- MMNodes.remove(leafMNode);
- if (MMNodes.isEmpty()) {
- tagVal2LeafMNodeSet.remove(value);
- if (tagVal2LeafMNodeSet.isEmpty()) {
- tagIndex.remove(key);
- }
- }
- }
- } else {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG_1, "Drop" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
- key,
- value,
- leafMNode.getOffset(),
- tagIndex.containsKey(key)));
- }
- }
- }
+ tagManager.dropTagsOrAttributes(keySet, fullPath, leafMNode);
}
/**
@@ -1767,76 +1512,18 @@ public class MManager {
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
public void setTagsOrAttributesValue(Map<String, String> alterMap, PartialPath fullPath)
throws MetadataException, IOException {
- MNode mNode = mtree.getNodeByPath(fullPath);
- if (!(mNode instanceof MeasurementMNode)) {
+ IMNode IMNode = mtree.getNodeByPath(fullPath);
+ if (!(IMNode instanceof MeasurementMNode)) {
throw new PathNotExistException(fullPath.getFullPath());
}
- MeasurementMNode leafMNode = (MeasurementMNode) mNode;
+ MeasurementMNode leafMNode = (MeasurementMNode) IMNode;
if (leafMNode.getOffset() < 0) {
throw new MetadataException(
String.format("TimeSeries [%s] does not have any tag/attribute.", fullPath));
}
// tags, attributes
- Pair<Map<String, String>, Map<String, String>> pair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
- Map<String, String> oldTagValue = new HashMap<>();
- Map<String, String> newTagValue = new HashMap<>();
-
- for (Entry<String, String> entry : alterMap.entrySet()) {
- String key = entry.getKey();
- String value = entry.getValue();
- // check tag map
- if (pair.left.containsKey(key)) {
- oldTagValue.put(key, pair.left.get(key));
- newTagValue.put(key, value);
- pair.left.put(key, value);
- } else if (pair.right.containsKey(key)) {
- // check attribute map
- pair.right.put(key, value);
- } else {
- throw new MetadataException(
- String.format("TimeSeries [%s] does not have tag/attribute [%s].", fullPath, key),
- true);
- }
- }
-
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
-
- for (Entry<String, String> entry : oldTagValue.entrySet()) {
- String key = entry.getKey();
- String beforeValue = entry.getValue();
- String currentValue = newTagValue.get(key);
- // change the tag inverted index map
- if (tagIndex.containsKey(key) && tagIndex.get(key).containsKey(beforeValue)) {
-
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG, "Set" + TAG_FORMAT, leafMNode.getFullPath()),
- entry.getKey(),
- beforeValue,
- leafMNode.getOffset()));
- }
-
- tagIndex.get(key).get(beforeValue).remove(leafMNode);
- } else {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG_1, "Set" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
- key,
- beforeValue,
- leafMNode.getOffset(),
- tagIndex.containsKey(key)));
- }
- }
- tagIndex
- .computeIfAbsent(key, k -> new ConcurrentHashMap<>())
- .computeIfAbsent(currentValue, k -> new CopyOnWriteArraySet<>())
- .add(leafMNode);
- }
+ tagManager.setTagsOrAttributesValue(alterMap, fullPath, leafMNode);
}
/**
@@ -1849,74 +1536,18 @@ public class MManager {
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
public void renameTagOrAttributeKey(String oldKey, String newKey, PartialPath fullPath)
throws MetadataException, IOException {
- MNode mNode = mtree.getNodeByPath(fullPath);
- if (!(mNode instanceof MeasurementMNode)) {
+ IMNode IMNode = mtree.getNodeByPath(fullPath);
+ if (!(IMNode instanceof MeasurementMNode)) {
throw new PathNotExistException(fullPath.getFullPath());
}
- MeasurementMNode leafMNode = (MeasurementMNode) mNode;
+ MeasurementMNode leafMNode = (MeasurementMNode) IMNode;
if (leafMNode.getOffset() < 0) {
throw new MetadataException(
String.format("TimeSeries [%s] does not have [%s] tag/attribute.", fullPath, oldKey),
true);
}
// tags, attributes
- Pair<Map<String, String>, Map<String, String>> pair =
- tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
-
- // current name has existed
- if (pair.left.containsKey(newKey) || pair.right.containsKey(newKey)) {
- throw new MetadataException(
- String.format(
- "TimeSeries [%s] already has a tag/attribute named [%s].", fullPath, newKey),
- true);
- }
-
- // check tag map
- if (pair.left.containsKey(oldKey)) {
- String value = pair.left.remove(oldKey);
- pair.left.put(newKey, value);
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
- // change the tag inverted index map
- if (tagIndex.containsKey(oldKey) && tagIndex.get(oldKey).containsKey(value)) {
-
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(DEBUG_MSG, "Rename" + TAG_FORMAT, leafMNode.getFullPath()),
- oldKey,
- value,
- leafMNode.getOffset()));
- }
-
- tagIndex.get(oldKey).get(value).remove(leafMNode);
-
- } else {
- if (logger.isDebugEnabled()) {
- logger.debug(
- String.format(
- String.format(
- DEBUG_MSG_1, "Rename" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
- oldKey,
- value,
- leafMNode.getOffset(),
- tagIndex.containsKey(oldKey)));
- }
- }
- tagIndex
- .computeIfAbsent(newKey, k -> new ConcurrentHashMap<>())
- .computeIfAbsent(value, k -> new CopyOnWriteArraySet<>())
- .add(leafMNode);
- } else if (pair.right.containsKey(oldKey)) {
- // check attribute map
- pair.right.put(newKey, pair.right.remove(oldKey));
- // persist the change to disk
- tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
- } else {
- throw new MetadataException(
- String.format("TimeSeries [%s] does not have tag/attribute [%s].", fullPath, oldKey),
- true);
- }
+ tagManager.renameTagOrAttributeKey(oldKey, newKey, fullPath, leafMNode);
}
/** Check whether the given path contains a storage group */
@@ -1939,11 +1570,11 @@ public class MManager {
}
public void collectTimeseriesSchema(
- MNode startingNode, Collection<TimeseriesSchema> timeseriesSchemas) {
- Deque<MNode> nodeDeque = new ArrayDeque<>();
+ IMNode startingNode, Collection<TimeseriesSchema> timeseriesSchemas) {
+ Deque<IMNode> nodeDeque = new ArrayDeque<>();
nodeDeque.addLast(startingNode);
while (!nodeDeque.isEmpty()) {
- MNode node = nodeDeque.removeFirst();
+ IMNode node = nodeDeque.removeFirst();
if (node instanceof MeasurementMNode) {
IMeasurementSchema nodeSchema = ((MeasurementMNode) node).getSchema();
timeseriesSchemas.add(
@@ -1964,11 +1595,11 @@ public class MManager {
}
public void collectMeasurementSchema(
- MNode startingNode, Collection<IMeasurementSchema> measurementSchemas) {
- Deque<MNode> nodeDeque = new ArrayDeque<>();
+ IMNode startingNode, Collection<IMeasurementSchema> measurementSchemas) {
+ Deque<IMNode> nodeDeque = new ArrayDeque<>();
nodeDeque.addLast(startingNode);
while (!nodeDeque.isEmpty()) {
- MNode node = nodeDeque.removeFirst();
+ IMNode node = nodeDeque.removeFirst();
if (node instanceof MeasurementMNode) {
IMeasurementSchema nodeSchema = ((MeasurementMNode) node).getSchema();
measurementSchemas.add(nodeSchema);
@@ -1980,13 +1611,13 @@ public class MManager {
/** Collect the timeseries schemas under "startingPath". */
public void collectSeries(PartialPath startingPath, List<IMeasurementSchema> measurementSchemas) {
- MNode mNode;
+ IMNode IMNode;
try {
- mNode = getNodeByPath(startingPath);
+ IMNode = getNodeByPath(startingPath);
} catch (MetadataException e) {
return;
}
- collectMeasurementSchema(mNode, measurementSchemas);
+ collectMeasurementSchema(IMNode, measurementSchemas);
}
/**
@@ -2058,68 +1689,9 @@ public class MManager {
return null;
}
- @TestOnly
- public void flushAllMlogForTest() throws IOException {
- logWriter.close();
- }
-
- private void checkMTreeModified() {
- if (logWriter == null || logFile == null) {
- // the logWriter is not initialized now, we skip the check once.
- return;
- }
- if (System.currentTimeMillis() - logFile.lastModified() < mtreeSnapshotThresholdTime) {
- if (logger.isDebugEnabled()) {
- logger.debug(
- "MTree snapshot need not be created. Time from last modification: {} ms.",
- System.currentTimeMillis() - logFile.lastModified());
- }
- } else if (logWriter.getLogNum() < mtreeSnapshotInterval) {
- if (logger.isDebugEnabled()) {
- logger.debug(
- "MTree snapshot need not be created. New mlog line number: {}.", logWriter.getLogNum());
- }
- } else {
- logger.info(
- "New mlog line number: {}, time from last modification: {} ms",
- logWriter.getLogNum(),
- System.currentTimeMillis() - logFile.lastModified());
- createMTreeSnapshot();
- }
- }
-
- public void createMTreeSnapshot() {
- long time = System.currentTimeMillis();
- logger.info("Start creating MTree snapshot to {}", mtreeSnapshotPath);
- try {
- mtree.serializeTo(mtreeSnapshotTmpPath);
- File tmpFile = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath);
- File snapshotFile = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotPath);
- if (snapshotFile.exists()) {
- Files.delete(snapshotFile.toPath());
- }
- if (tmpFile.renameTo(snapshotFile)) {
- logger.info(
- "Finish creating MTree snapshot to {}, spend {} ms.",
- mtreeSnapshotPath,
- System.currentTimeMillis() - time);
- }
- logWriter.clear();
- } catch (IOException e) {
- logger.warn("Failed to create MTree snapshot to {}", mtreeSnapshotPath, e);
- if (SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath).exists()) {
- try {
- Files.delete(SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath).toPath());
- } catch (IOException e1) {
- logger.warn("delete file {} failed: {}", mtreeSnapshotTmpPath, e1.getMessage());
- }
- }
- }
- }
-
/** get schema for device. Attention!!! Only support insertPlan */
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
- public MNode getSeriesSchemasAndReadLockDevice(InsertPlan plan)
+ public IMNode getSeriesSchemasAndReadLockDevice(InsertPlan plan)
throws MetadataException, IOException {
PartialPath prefixPath = plan.getPrefixPath();
PartialPath deviceId = prefixPath;
@@ -2132,8 +1704,9 @@ public class MManager {
MeasurementMNode[] measurementMNodes = plan.getMeasurementMNodes();
// 1. get device node
- Pair<MNode, Template> deviceMNode = getDeviceNodeWithAutoCreate(deviceId);
- if (deviceMNode.left.getDeviceTemplate() != null) {
+ Pair<IMNode, Template> deviceMNode = getDeviceNodeWithAutoCreate(deviceId);
+ if (!(deviceMNode.left instanceof MeasurementMNode)
+ && deviceMNode.left.getDeviceTemplate() != null) {
deviceMNode.right = deviceMNode.left.getDeviceTemplate();
}
@@ -2162,7 +1735,7 @@ public class MManager {
for (int i = 0; i < measurementList.length; i++) {
try {
String measurement = measurementList[i];
- MNode child = getMNode(deviceMNode.left, plan.isAligned() ? vectorId : measurement);
+ IMNode child = getMNode(deviceMNode.left, plan.isAligned() ? vectorId : measurement);
if (child instanceof MeasurementMNode) {
measurementMNode = (MeasurementMNode) child;
} else if (child instanceof StorageGroupMNode) {
@@ -2292,12 +1865,12 @@ public class MManager {
return dataType;
}
- public MNode getMNode(MNode deviceMNode, String measurementName) {
+ public IMNode getMNode(IMNode deviceMNode, String measurementName) {
return deviceMNode.getChild(measurementName);
}
private MeasurementMNode findTemplate(
- Pair<MNode, Template> deviceMNode, String measurement, String vectorId)
+ Pair<IMNode, Template> deviceMNode, String measurement, String vectorId)
throws MetadataException {
if (deviceMNode.right != null) {
Map<String, IMeasurementSchema> curTemplateMap = deviceMNode.right.getSchemaMap();
@@ -2364,12 +1937,7 @@ public class MManager {
public void createDeviceTemplate(CreateTemplatePlan plan) throws MetadataException {
try {
- Template template = new Template(plan);
- if (templateMap.putIfAbsent(plan.getName(), template) != null) {
- // already have template
- throw new MetadataException("Duplicated template name: " + plan.getName());
- }
-
+ templateManager.createDeviceTemplate(plan);
// write wal
if (!isRecovering) {
logWriter.createDeviceTemplate(plan);
@@ -2381,30 +1949,13 @@ public class MManager {
public void setDeviceTemplate(SetDeviceTemplatePlan plan) throws MetadataException {
try {
- Template template = templateMap.get(plan.getTemplateName());
-
- if (template == null) {
- throw new UndefinedTemplateException(plan.getTemplateName());
- }
+ Template template = templateManager.getTemplate(plan.getTemplateName());
// get mnode and update template should be atomic
synchronized (this) {
- Pair<MNode, Template> node =
+ Pair<IMNode, Template> node =
getDeviceNodeWithAutoCreate(new PartialPath(plan.getPrefixPath()));
-
- if (node.left.getDeviceTemplate() != null) {
- if (node.left.getDeviceTemplate().equals(template)) {
- throw new DuplicatedTemplateException(template.getName());
- } else {
- throw new MetadataException("Specified node already has template");
- }
- }
-
- if (!isTemplateCompatible(node.right, template)) {
- throw new MetadataException("Incompatible template");
- }
-
- node.left.setDeviceTemplate(template);
+ templateManager.setDeviceTemplate(template, node);
}
// write wal
@@ -2417,35 +1968,7 @@ public class MManager {
}
public boolean isTemplateCompatible(Template upper, Template current) {
- if (upper == null) {
- return true;
- }
-
- Map<String, IMeasurementSchema> upperMap = new HashMap<>(upper.getSchemaMap());
- Map<String, IMeasurementSchema> currentMap = new HashMap<>(current.getSchemaMap());
-
- // for identical vector schema, we should just compare once
- Map<IMeasurementSchema, IMeasurementSchema> sameSchema = new HashMap<>();
-
- for (String name : currentMap.keySet()) {
- IMeasurementSchema upperSchema = upperMap.remove(name);
- if (upperSchema != null) {
- IMeasurementSchema currentSchema = currentMap.get(name);
- // use "==" to compare actual address space
- if (upperSchema == sameSchema.get(currentSchema)) {
- continue;
- }
-
- if (!upperSchema.equals(currentSchema)) {
- return false;
- }
-
- sameSchema.put(currentSchema, upperSchema);
- }
- }
-
- // current template must contains all measurements of upper template
- return upperMap.isEmpty();
+ return templateManager.isTemplateCompatible(upper, current);
}
public void autoCreateDeviceMNode(AutoCreateDeviceMNodePlan plan) throws MetadataException {
@@ -2467,4 +1990,9 @@ public class MManager {
public long getTotalSeriesNumber() {
return totalSeriesNumber.get();
}
+
+ @TestOnly
+ public void flushAllMlogForTest() throws IOException {
+ logWriter.close();
+ }
}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/MTree.java b/server/src/main/java/org/apache/iotdb/db/metadata/MTree.java
index f18f689..aff1214 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/MTree.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/MTree.java
@@ -21,6 +21,7 @@ package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.conf.IoTDBConfig;
import org.apache.iotdb.db.conf.IoTDBConstant;
import org.apache.iotdb.db.conf.IoTDBDescriptor;
+import org.apache.iotdb.db.engine.fileSystem.SystemFileFactory;
import org.apache.iotdb.db.engine.querycontext.QueryDataSource;
import org.apache.iotdb.db.exception.metadata.AliasAlreadyExistException;
import org.apache.iotdb.db.exception.metadata.IllegalParameterOfPathException;
@@ -33,7 +34,8 @@ import org.apache.iotdb.db.exception.metadata.StorageGroupNotSetException;
import org.apache.iotdb.db.metadata.MManager.StorageGroupFilter;
import org.apache.iotdb.db.metadata.logfile.MLogReader;
import org.apache.iotdb.db.metadata.logfile.MLogWriter;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
+import org.apache.iotdb.db.metadata.mnode.InternalMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.metadata.mnode.StorageGroupMNode;
import org.apache.iotdb.db.metadata.template.Template;
@@ -67,6 +69,7 @@ import org.slf4j.LoggerFactory;
import java.io.File;
import java.io.IOException;
import java.io.Serializable;
+import java.nio.file.Files;
import java.util.ArrayDeque;
import java.util.ArrayList;
import java.util.Comparator;
@@ -104,17 +107,78 @@ public class MTree implements Serializable {
private static transient ThreadLocal<Integer> offset = new ThreadLocal<>();
private static transient ThreadLocal<Integer> count = new ThreadLocal<>();
private static transient ThreadLocal<Integer> curOffset = new ThreadLocal<>();
- private MNode root;
+ private IMNode root;
+
+ private String mtreeSnapshotPath;
+ private String mtreeSnapshotTmpPath;
MTree() {
- this.root = new MNode(null, IoTDBConstant.PATH_ROOT);
+ this.root = new InternalMNode(null, IoTDBConstant.PATH_ROOT);
}
- private MTree(MNode root) {
+ private MTree(InternalMNode root) {
this.root = root;
}
- static long getLastTimeStamp(MeasurementMNode node, QueryContext queryContext) {
+ public void init() throws IOException {
+ mtreeSnapshotPath =
+ IoTDBDescriptor.getInstance().getConfig().getSchemaDir()
+ + File.separator
+ + MetadataConstant.MTREE_SNAPSHOT;
+ mtreeSnapshotTmpPath =
+ IoTDBDescriptor.getInstance().getConfig().getSchemaDir()
+ + File.separator
+ + MetadataConstant.MTREE_SNAPSHOT_TMP;
+
+ File tmpFile = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath);
+ if (tmpFile.exists()) {
+ logger.warn("Creating MTree snapshot not successful before crashing...");
+ Files.delete(tmpFile.toPath());
+ }
+
+ File mtreeSnapshot = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotPath);
+ long time = System.currentTimeMillis();
+ if (mtreeSnapshot.exists()) {
+ this.root = deserializeFrom(mtreeSnapshot).root;
+ logger.debug(
+ "spend {} ms to deserialize mtree from snapshot", System.currentTimeMillis() - time);
+ }
+ }
+
+ public void clear() {
+ root = new InternalMNode(null, IoTDBConstant.PATH_ROOT);
+ }
+
+ public void createSnapshot() throws IOException {
+ long time = System.currentTimeMillis();
+ logger.info("Start creating MTree snapshot to {}", mtreeSnapshotPath);
+ try {
+ serializeTo(mtreeSnapshotTmpPath);
+ File tmpFile = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath);
+ File snapshotFile = SystemFileFactory.INSTANCE.getFile(mtreeSnapshotPath);
+ if (snapshotFile.exists()) {
+ Files.delete(snapshotFile.toPath());
+ }
+ if (tmpFile.renameTo(snapshotFile)) {
+ logger.info(
+ "Finish creating MTree snapshot to {}, spend {} ms.",
+ mtreeSnapshotPath,
+ System.currentTimeMillis() - time);
+ }
+ } catch (IOException e) {
+ logger.warn("Failed to create MTree snapshot to {}", mtreeSnapshotPath, e);
+ if (SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath).exists()) {
+ try {
+ Files.delete(SystemFileFactory.INSTANCE.getFile(mtreeSnapshotTmpPath).toPath());
+ } catch (IOException e1) {
+ logger.warn("delete file {} failed: {}", mtreeSnapshotTmpPath, e1.getMessage());
+ }
+ }
+ throw e;
+ }
+ }
+
+ public static long getLastTimeStamp(MeasurementMNode node, QueryContext queryContext) {
TimeValuePair last = node.getCachedLast();
if (last != null) {
return node.getCachedLast().getTimestamp();
@@ -219,28 +283,39 @@ public class MTree implements Serializable {
throw new IllegalPathException(path.getFullPath());
}
checkTimeseries(path.getFullPath());
- MNode cur = root;
+ IMNode cur = root;
boolean hasSetStorageGroup = false;
Template upperTemplate = cur.getDeviceTemplate();
// e.g, path = root.sg.d1.s1, create internal nodes and set cur to d1 node
for (int i = 1; i < nodeNames.length - 1; i++) {
- String nodeName = nodeNames[i];
+ if (cur instanceof MeasurementMNode) {
+ throw new PathAlreadyExistException(cur.getFullPath());
+ }
if (cur instanceof StorageGroupMNode) {
hasSetStorageGroup = true;
}
- if (!cur.hasChild(nodeName)) {
+ String childName = nodeNames[i];
+ if (!cur.hasChild(childName)) {
if (!hasSetStorageGroup) {
throw new StorageGroupNotSetException("Storage group should be created first");
}
- cur.addChild(nodeName, new MNode(cur, nodeName));
+ if (cur.isUseTemplate() && upperTemplate.hasSchema(childName)) {
+ throw new PathAlreadyExistException(
+ cur.getPartialPath().concatNode(childName).getFullPath());
+ }
+ cur.addChild(childName, new InternalMNode(cur, childName));
}
- cur = cur.getChild(nodeName);
+ cur = cur.getChild(childName);
if (cur.getDeviceTemplate() != null) {
upperTemplate = cur.getDeviceTemplate();
}
}
+ if (cur instanceof MeasurementMNode) {
+ throw new PathAlreadyExistException(cur.getFullPath());
+ }
+
if (upperTemplate != null && !upperTemplate.isCompatible(path)) {
throw new PathAlreadyExistException(
path.getFullPath() + " ( which is incompatible with template )");
@@ -255,32 +330,21 @@ public class MTree implements Serializable {
// synchronize check and add, we need addChild and add Alias become atomic operation
// only write on mtree will be synchronized
synchronized (this) {
- MNode child = cur.getChild(leafName);
- if (child instanceof MeasurementMNode || child instanceof StorageGroupMNode) {
+ if (cur.hasChild(leafName)) {
throw new PathAlreadyExistException(path.getFullPath());
}
- if (alias != null) {
- MNode childByAlias = cur.getChild(alias);
- if (childByAlias instanceof MeasurementMNode) {
- throw new AliasAlreadyExistException(path.getFullPath(), alias);
- }
+ if (alias != null && cur.hasChild(alias)) {
+ throw new AliasAlreadyExistException(path.getFullPath(), alias);
}
- // this measurementMNode could be a leaf or not.
MeasurementMNode measurementMNode =
new MeasurementMNode(cur, leafName, alias, dataType, encoding, compressor, props);
- if (child != null) {
- cur.replaceChild(measurementMNode.getName(), measurementMNode);
- } else {
- cur.addChild(leafName, measurementMNode);
- }
-
+ cur.addChild(leafName, measurementMNode);
// link alias to LeafMNode
if (alias != null) {
cur.addAlias(alias, measurementMNode);
}
-
return measurementMNode;
}
}
@@ -310,29 +374,36 @@ public class MTree implements Serializable {
for (String measurement : measurements) {
checkTimeseries(measurement);
}
- MNode cur = root;
+ IMNode cur = root;
boolean hasSetStorageGroup = false;
// e.g, devicePath = root.sg.d1, create internal nodes and set cur to d1 node
for (int i = 1; i < deviceNodeNames.length - 1; i++) {
- String nodeName = deviceNodeNames[i];
+ if (cur instanceof MeasurementMNode) {
+ throw new PathAlreadyExistException(cur.getFullPath());
+ }
if (cur instanceof StorageGroupMNode) {
hasSetStorageGroup = true;
}
+ String nodeName = deviceNodeNames[i];
if (!cur.hasChild(nodeName)) {
if (!hasSetStorageGroup) {
throw new StorageGroupNotSetException("Storage group should be created first");
}
- cur.addChild(nodeName, new MNode(cur, nodeName));
+ cur.addChild(nodeName, new InternalMNode(cur, nodeName));
}
cur = cur.getChild(nodeName);
}
+
+ if (cur instanceof MeasurementMNode) {
+ throw new PathAlreadyExistException(cur.getFullPath());
+ }
+
String leafName = deviceNodeNames[deviceNodeNames.length - 1];
// synchronize check and add, we need addChild and add Alias become atomic operation
// only write on mtree will be synchronized
synchronized (this) {
- MNode child = cur.getChild(leafName);
- if (child instanceof MeasurementMNode || child instanceof StorageGroupMNode) {
+ if (cur.hasChild(leafName)) {
throw new PathAlreadyExistException(devicePath.getFullPath() + "." + leafName);
}
@@ -351,14 +422,6 @@ public class MTree implements Serializable {
compressor),
null);
cur.addChild(leafName, measurementMNode);
-
- for (String measurement : measurements) {
- if (child != null) {
- measurementMNode.replaceChild(measurement, new MNode(measurementMNode, measurement));
- } else {
- measurementMNode.addChild(measurement, new MNode(measurementMNode, measurement));
- }
- }
}
}
@@ -422,28 +485,32 @@ public class MTree implements Serializable {
*
* <p>e.g., get root.sg.d1, get or create all internal nodes and return the node of d1
*/
- Pair<MNode, Template> getDeviceNodeWithAutoCreating(PartialPath deviceId, int sgLevel)
+ Pair<IMNode, Template> getDeviceNodeWithAutoCreating(PartialPath deviceId, int sgLevel)
throws MetadataException {
String[] nodeNames = deviceId.getNodes();
if (nodeNames.length <= 1 || !nodeNames[0].equals(root.getName())) {
throw new IllegalPathException(deviceId.getFullPath());
}
- MNode cur = root;
- Template upperTemplate = null;
+ IMNode cur = root;
+ Template upperTemplate = cur.getDeviceTemplate();
for (int i = 1; i < nodeNames.length; i++) {
if (!cur.hasChild(nodeNames[i])) {
+ if (cur.isUseTemplate() && upperTemplate.hasSchema(nodeNames[i])) {
+ throw new PathAlreadyExistException(
+ cur.getPartialPath().concatNode(nodeNames[i]).getFullPath());
+ }
if (i == sgLevel) {
cur.addChild(
nodeNames[i],
new StorageGroupMNode(
cur, nodeNames[i], IoTDBDescriptor.getInstance().getConfig().getDefaultTTL()));
} else {
- cur.addChild(nodeNames[i], new MNode(cur, nodeNames[i]));
+ cur.addChild(nodeNames[i], new InternalMNode(cur, nodeNames[i]));
}
}
+ cur = cur.getChild(nodeNames[i]);
// update upper template
upperTemplate = cur.getDeviceTemplate() == null ? upperTemplate : cur.getDeviceTemplate();
- cur = cur.getChild(nodeNames[i]);
}
return new Pair<>(cur, upperTemplate);
@@ -456,19 +523,28 @@ public class MTree implements Serializable {
*/
boolean isPathExist(PartialPath path) {
String[] nodeNames = path.getNodes();
- MNode cur = root;
+ IMNode cur = root;
if (!nodeNames[0].equals(root.getName())) {
return false;
}
+ Template upperTemplate = cur.getDeviceTemplate();
for (int i = 1; i < nodeNames.length; i++) {
- String childName = nodeNames[i];
- cur = cur.getChild(childName);
- if (cur instanceof MeasurementMNode
- && ((MeasurementMNode) cur).getSchema() instanceof VectorMeasurementSchema) {
- return i == nodeNames.length - 1 || cur.getChildren().containsKey(nodeNames[i + 1]);
- } else if (cur == null) {
- return false;
+ if (!cur.hasChild(nodeNames[i])) {
+ return cur.isUseTemplate() && upperTemplate.hasSchema(nodeNames[i]);
+ }
+ cur = cur.getChild(nodeNames[i]);
+ if (cur instanceof MeasurementMNode) {
+ if (i == nodeNames.length - 1) {
+ return true;
+ }
+ if (((MeasurementMNode) cur).getSchema() instanceof VectorMeasurementSchema) {
+ return i == nodeNames.length - 2
+ && ((MeasurementMNode) cur).getSchema().isCompatible(nodeNames[i + 1]);
+ } else {
+ return false;
+ }
}
+ upperTemplate = cur.getDeviceTemplate() == null ? upperTemplate : cur.getDeviceTemplate();
}
return true;
}
@@ -481,21 +557,27 @@ public class MTree implements Serializable {
void setStorageGroup(PartialPath path) throws MetadataException {
String[] nodeNames = path.getNodes();
checkStorageGroup(path.getFullPath());
- MNode cur = root;
if (nodeNames.length <= 1 || !nodeNames[0].equals(root.getName())) {
throw new IllegalPathException(path.getFullPath());
}
+ IMNode cur = root;
+ Template upperTemplate = cur.getDeviceTemplate();
int i = 1;
// e.g., path = root.a.b.sg, create internal nodes for a, b
while (i < nodeNames.length - 1) {
- MNode temp = cur.getChild(nodeNames[i]);
+ IMNode temp = cur.getChild(nodeNames[i]);
if (temp == null) {
- cur.addChild(nodeNames[i], new MNode(cur, nodeNames[i]));
+ if (cur.isUseTemplate() && upperTemplate.hasSchema(nodeNames[i])) {
+ throw new PathAlreadyExistException(
+ cur.getPartialPath().concatNode(nodeNames[i]).getFullPath());
+ }
+ cur.addChild(nodeNames[i], new InternalMNode(cur, nodeNames[i]));
} else if (temp instanceof StorageGroupMNode) {
// before set storage group, check whether the exists or not
throw new StorageGroupAlreadySetException(temp.getFullPath());
}
cur = cur.getChild(nodeNames[i]);
+ upperTemplate = cur.getDeviceTemplate() == null ? upperTemplate : cur.getDeviceTemplate();
i++;
}
@@ -510,6 +592,10 @@ public class MTree implements Serializable {
throw new StorageGroupAlreadySetException(path.getFullPath(), true);
}
} else {
+ if (cur.isUseTemplate() && upperTemplate.hasSchema(nodeNames[i])) {
+ throw new PathAlreadyExistException(
+ cur.getPartialPath().concatNode(nodeNames[i]).getFullPath());
+ }
StorageGroupMNode storageGroupMNode =
new StorageGroupMNode(
cur, nodeNames[i], IoTDBDescriptor.getInstance().getConfig().getDefaultTTL());
@@ -529,7 +615,7 @@ public class MTree implements Serializable {
/** Delete a storage group */
List<MeasurementMNode> deleteStorageGroup(PartialPath path) throws MetadataException {
- MNode cur = getNodeByPath(path);
+ IMNode cur = getNodeByPath(path);
if (!(cur instanceof StorageGroupMNode)) {
throw new StorageGroupNotSetException(path.getFullPath());
}
@@ -539,11 +625,11 @@ public class MTree implements Serializable {
// collect all the LeafMNode in this storage group
List<MeasurementMNode> leafMNodes = new LinkedList<>();
- Queue<MNode> queue = new LinkedList<>();
+ Queue<IMNode> queue = new LinkedList<>();
queue.add(cur);
while (!queue.isEmpty()) {
- MNode node = queue.poll();
- for (MNode child : node.getChildren().values()) {
+ IMNode node = queue.poll();
+ for (IMNode child : node.getChildren().values()) {
if (child instanceof MeasurementMNode) {
leafMNodes.add((MeasurementMNode) child);
} else {
@@ -575,7 +661,7 @@ public class MTree implements Serializable {
if (nodeNames.length <= 1 || !nodeNames[0].equals(IoTDBConstant.PATH_ROOT)) {
return false;
}
- MNode cur = root;
+ IMNode cur = root;
int i = 1;
while (i < nodeNames.length - 1) {
cur = cur.getChild(nodeNames[i]);
@@ -595,7 +681,7 @@ public class MTree implements Serializable {
*/
Pair<PartialPath, MeasurementMNode> deleteTimeseriesAndReturnEmptyStorageGroup(PartialPath path)
throws MetadataException {
- MNode curNode = getNodeByPath(path);
+ IMNode curNode = getNodeByPath(path);
if (!(curNode instanceof MeasurementMNode)) {
throw new PathNotExistException(path.getFullPath());
}
@@ -638,7 +724,7 @@ public class MTree implements Serializable {
* Get node by path with storage group check If storage group is not set,
* StorageGroupNotSetException will be thrown
*/
- Pair<MNode, Template> getNodeByPathWithStorageGroupCheck(PartialPath path)
+ Pair<IMNode, Template> getNodeByPathWithStorageGroupCheck(PartialPath path)
throws MetadataException {
boolean storageGroupChecked = false;
String[] nodes = path.getNodes();
@@ -646,7 +732,7 @@ public class MTree implements Serializable {
throw new IllegalPathException(path.getFullPath());
}
- MNode cur = root;
+ IMNode cur = root;
Template upperTemplate = null;
for (int i = 1; i < nodes.length; i++) {
@@ -678,7 +764,7 @@ public class MTree implements Serializable {
*/
StorageGroupMNode getStorageGroupNodeByStorageGroupPath(PartialPath path)
throws MetadataException {
- MNode node = getNodeByPath(path);
+ IMNode node = getNodeByPath(path);
if (node instanceof StorageGroupMNode) {
return (StorageGroupMNode) node;
} else {
@@ -696,7 +782,7 @@ public class MTree implements Serializable {
if (nodes.length == 0 || !nodes[0].equals(root.getName())) {
throw new IllegalPathException(path.getFullPath());
}
- MNode cur = root;
+ IMNode cur = root;
for (int i = 1; i < nodes.length; i++) {
if (cur == null) {
break;
@@ -714,23 +800,27 @@ public class MTree implements Serializable {
*
* @return last node in given seriesPath
*/
- MNode getNodeByPath(PartialPath path) throws MetadataException {
+ IMNode getNodeByPath(PartialPath path) throws MetadataException {
String[] nodes = path.getNodes();
if (nodes.length == 0 || !nodes[0].equals(root.getName())) {
throw new IllegalPathException(path.getFullPath());
}
- MNode cur = root;
+ IMNode cur = root;
Template upperTemplate = cur.getDeviceTemplate();
for (int i = 1; i < nodes.length; i++) {
+ if (cur instanceof MeasurementMNode) {
+ if (i == nodes.length - 1
+ || ((MeasurementMNode) cur).getSchema() instanceof VectorMeasurementSchema) {
+ return cur;
+ } else {
+ throw new PathNotExistException(path.getFullPath(), true);
+ }
+ }
if (cur.getDeviceTemplate() != null) {
upperTemplate = cur.getDeviceTemplate();
}
- MNode next = cur.getChild(nodes[i]);
- if (cur instanceof MeasurementMNode
- && ((MeasurementMNode) cur).getSchema() instanceof VectorMeasurementSchema) {
- return cur;
- }
+ IMNode next = cur.getChild(nodes[i]);
if (next == null) {
if (upperTemplate == null) {
throw new PathNotExistException(path.getFullPath(), true);
@@ -770,20 +860,20 @@ public class MTree implements Serializable {
* @apiNote :for cluster
*/
private void findStorageGroup(
- MNode node, String[] nodes, int idx, String parent, List<String> storageGroupNames) {
+ IMNode node, String[] nodes, int idx, String parent, List<String> storageGroupNames) {
if (node instanceof StorageGroupMNode) {
storageGroupNames.add(node.getFullPath());
return;
}
String nodeReg = MetaUtils.getNodeRegByIdx(idx, nodes);
if (!(PATH_WILDCARD).equals(nodeReg)) {
- MNode next = node.getChild(nodeReg);
+ IMNode next = node.getChild(nodeReg);
if (next != null) {
findStorageGroup(
next, nodes, idx + 1, parent + node.getName() + PATH_SEPARATOR, storageGroupNames);
}
} else {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
findStorageGroup(
child, nodes, idx + 1, parent + node.getName() + PATH_SEPARATOR, storageGroupNames);
}
@@ -797,10 +887,10 @@ public class MTree implements Serializable {
*/
List<PartialPath> getAllStorageGroupPaths() {
List<PartialPath> res = new ArrayList<>();
- Deque<MNode> nodeStack = new ArrayDeque<>();
+ Deque<IMNode> nodeStack = new ArrayDeque<>();
nodeStack.add(root);
while (!nodeStack.isEmpty()) {
- MNode current = nodeStack.pop();
+ IMNode current = nodeStack.pop();
if (current instanceof StorageGroupMNode) {
res.add(current.getPartialPath());
} else {
@@ -859,7 +949,7 @@ public class MTree implements Serializable {
* @param prefixOnly only return storage groups that start with this prefix path
*/
private void findStorageGroupPaths(
- MNode node,
+ IMNode node,
String[] nodes,
int idx,
String parent,
@@ -871,7 +961,7 @@ public class MTree implements Serializable {
}
String nodeReg = MetaUtils.getNodeRegByIdx(idx, nodes);
if (!(PATH_WILDCARD).equals(nodeReg)) {
- MNode next = node.getChild(nodeReg);
+ IMNode next = node.getChild(nodeReg);
if (next != null) {
findStorageGroupPaths(
node.getChild(nodeReg),
@@ -882,7 +972,7 @@ public class MTree implements Serializable {
prefixOnly);
}
} else {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
findStorageGroupPaths(
child,
nodes,
@@ -897,10 +987,10 @@ public class MTree implements Serializable {
/** Get all storage group MNodes */
List<StorageGroupMNode> getAllStorageGroupNodes() {
List<StorageGroupMNode> ret = new ArrayList<>();
- Deque<MNode> nodeStack = new ArrayDeque<>();
+ Deque<IMNode> nodeStack = new ArrayDeque<>();
nodeStack.add(root);
while (!nodeStack.isEmpty()) {
- MNode current = nodeStack.pop();
+ IMNode current = nodeStack.pop();
if (current instanceof StorageGroupMNode) {
ret.add((StorageGroupMNode) current);
} else {
@@ -919,7 +1009,7 @@ public class MTree implements Serializable {
*/
PartialPath getStorageGroupPath(PartialPath path) throws StorageGroupNotSetException {
String[] nodes = path.getNodes();
- MNode cur = root;
+ IMNode cur = root;
for (int i = 1; i < nodes.length; i++) {
cur = cur.getChild(nodes[i]);
if (cur instanceof StorageGroupMNode) {
@@ -934,7 +1024,7 @@ public class MTree implements Serializable {
/** Check whether the given path contains a storage group */
boolean checkStorageGroupByPath(PartialPath path) {
String[] nodes = path.getNodes();
- MNode cur = root;
+ IMNode cur = root;
for (int i = 1; i < nodes.length; i++) {
cur = cur.getChild(nodes[i]);
if (cur == null) {
@@ -1010,17 +1100,32 @@ public class MTree implements Serializable {
}
/** Traverse the MTree to get the count of timeseries. */
- private int getCount(MNode node, String[] nodes, int idx, boolean wildcard)
+ private int getCount(IMNode node, String[] nodes, int idx, boolean wildcard)
throws PathNotExistException {
+ if (node instanceof MeasurementMNode) {
+ if (idx < nodes.length) {
+ if (((MeasurementMNode) node).getSchema().isCompatible(nodes[idx])) {
+ return 1;
+ } else {
+ if (!wildcard) {
+ throw new PathNotExistException(node.getName() + NO_CHILDNODE_MSG + nodes[idx]);
+ } else {
+ return 0;
+ }
+ }
+ } else {
+ return ((MeasurementMNode) node).getMeasurementCount();
+ }
+ }
if (idx < nodes.length) {
if (PATH_WILDCARD.equals(nodes[idx])) {
int sum = 0;
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
sum += getCount(child, nodes, idx + 1, true);
}
return sum;
} else {
- MNode child = node.getChild(nodes[idx]);
+ IMNode child = node.getChild(nodes[idx]);
if (child == null) {
if (node.isUseTemplate()
&& node.getUpperTemplate().getSchemaMap().containsKey(nodes[idx])) {
@@ -1035,11 +1140,11 @@ public class MTree implements Serializable {
return getCount(child, nodes, idx + 1, wildcard);
}
} else {
- int sum = node instanceof MeasurementMNode ? 1 : 0;
+ int sum = 0;
if (node.isUseTemplate()) {
sum += node.getUpperTemplate().getSchemaMap().size();
}
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
sum += getCount(child, nodes, idx + 1, wildcard);
}
return sum;
@@ -1078,7 +1183,7 @@ public class MTree implements Serializable {
if (nodes.length == 0 || !nodes[0].equals(root.getName())) {
throw new IllegalPathException(prefixPath.getFullPath());
}
- MNode node = root;
+ IMNode node = root;
int i;
for (i = 1; i < nodes.length; i++) {
if (nodes[i].equals("*")) {
@@ -1094,12 +1199,12 @@ public class MTree implements Serializable {
}
/** Traverse the MTree to get the count of devices. */
- private int getDevicesCount(MNode node, String[] nodes, int idx) {
+ private int getDevicesCount(IMNode node, String[] nodes, int idx) {
String nodeReg = MetaUtils.getNodeRegByIdx(idx, nodes);
boolean curIsDevice = node.isUseTemplate();
int cnt = curIsDevice ? 1 : 0;
if (!(PATH_WILDCARD).equals(nodeReg)) {
- MNode next = node.getChild(nodeReg);
+ IMNode next = node.getChild(nodeReg);
if (next != null) {
if (next instanceof MeasurementMNode && idx >= nodes.length && !curIsDevice) {
cnt++;
@@ -1108,7 +1213,7 @@ public class MTree implements Serializable {
}
}
} else {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
if (child instanceof MeasurementMNode && !curIsDevice && idx >= nodes.length) {
cnt++;
curIsDevice = true;
@@ -1120,7 +1225,7 @@ public class MTree implements Serializable {
}
/** Traverse the MTree to get the count of storage group. */
- private int getStorageGroupCount(MNode node, String[] nodes, int idx, String parent) {
+ private int getStorageGroupCount(IMNode node, String[] nodes, int idx, String parent) {
int cnt = 0;
if (node instanceof StorageGroupMNode && idx >= nodes.length) {
cnt++;
@@ -1128,12 +1233,12 @@ public class MTree implements Serializable {
}
String nodeReg = MetaUtils.getNodeRegByIdx(idx, nodes);
if (!(PATH_WILDCARD).equals(nodeReg)) {
- MNode next = node.getChild(nodeReg);
+ IMNode next = node.getChild(nodeReg);
if (next != null) {
cnt += getStorageGroupCount(next, nodes, idx + 1, parent + node.getName() + PATH_SEPARATOR);
}
} else {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
cnt +=
getStorageGroupCount(child, nodes, idx + 1, parent + node.getName() + PATH_SEPARATOR);
}
@@ -1146,12 +1251,12 @@ public class MTree implements Serializable {
*
* @param targetLevel Record the distance to the target level, 0 means the target level.
*/
- private int getCountInGivenLevel(MNode node, int targetLevel) {
+ private int getCountInGivenLevel(IMNode node, int targetLevel) {
if (targetLevel == 0) {
return 1;
}
int cnt = 0;
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
cnt += getCountInGivenLevel(child, targetLevel - 1);
}
return cnt;
@@ -1229,7 +1334,7 @@ public class MTree implements Serializable {
*/
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
private void findPath(
- MNode node,
+ IMNode node,
String[] nodes,
int idx,
List<Pair<PartialPath, String[]>> timeseriesSchemaList,
@@ -1238,31 +1343,33 @@ public class MTree implements Serializable {
QueryContext queryContext,
Template upperTemplate)
throws MetadataException {
- if (node instanceof MeasurementMNode
- && (nodes.length <= idx
- || ((MeasurementMNode) node).getSchema() instanceof VectorMeasurementSchema)) {
- if (hasLimit) {
- curOffset.set(curOffset.get() + 1);
- if (curOffset.get() < offset.get() || count.get().intValue() == limit.get().intValue()) {
- return;
+ if (node instanceof MeasurementMNode) {
+ if ((nodes.length <= idx
+ || ((MeasurementMNode) node).getSchema() instanceof VectorMeasurementSchema)) {
+ if (hasLimit) {
+ curOffset.set(curOffset.get() + 1);
+ if (curOffset.get() < offset.get() || count.get().intValue() == limit.get().intValue()) {
+ return;
+ }
+ }
+ IMeasurementSchema measurementSchema = ((MeasurementMNode) node).getSchema();
+ if (measurementSchema instanceof MeasurementSchema) {
+ addMeasurementSchema(
+ node, timeseriesSchemaList, needLast, queryContext, measurementSchema, "*");
+ } else if (measurementSchema instanceof VectorMeasurementSchema) {
+ addVectorMeasurementSchema(
+ node,
+ timeseriesSchemaList,
+ needLast,
+ queryContext,
+ measurementSchema,
+ idx < nodes.length ? nodes[idx] : "*");
+ }
+ if (hasLimit) {
+ count.set(count.get() + 1);
}
}
- IMeasurementSchema measurementSchema = ((MeasurementMNode) node).getSchema();
- if (measurementSchema instanceof MeasurementSchema) {
- addMeasurementSchema(
- node, timeseriesSchemaList, needLast, queryContext, measurementSchema, "*");
- } else if (measurementSchema instanceof VectorMeasurementSchema) {
- addVectorMeasurementSchema(
- node,
- timeseriesSchemaList,
- needLast,
- queryContext,
- measurementSchema,
- idx < nodes.length ? nodes[idx] : "*");
- }
- if (hasLimit) {
- count.set(count.get() + 1);
- }
+ return;
}
String nodeReg = MetaUtils.getNodeRegByIdx(idx, nodes);
@@ -1272,7 +1379,7 @@ public class MTree implements Serializable {
// we should use template when all child is measurement or this node has no child
if (!nodeReg.contains(PATH_WILDCARD)) {
- MNode next = node.getChild(nodeReg);
+ IMNode next = node.getChild(nodeReg);
if (next != null) {
findPath(
next,
@@ -1285,7 +1392,7 @@ public class MTree implements Serializable {
upperTemplate);
}
} else {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
if (!Pattern.matches(nodeReg.replace("*", ".*"), child.getName())) {
continue;
}
@@ -1305,7 +1412,7 @@ public class MTree implements Serializable {
}
// template part
- if (!(node instanceof MeasurementMNode) && node.isUseTemplate()) {
+ if (node.isUseTemplate()) {
if (upperTemplate != null) {
HashSet<IMeasurementSchema> set = new HashSet<>();
for (IMeasurementSchema schema : upperTemplate.getSchemaMap().values()) {
@@ -1340,7 +1447,7 @@ public class MTree implements Serializable {
}
private void addMeasurementSchema(
- MNode node,
+ IMNode node,
List<Pair<PartialPath, String[]>> timeseriesSchemaList,
boolean needLast,
QueryContext queryContext,
@@ -1364,7 +1471,7 @@ public class MTree implements Serializable {
}
private void addVectorMeasurementSchema(
- MNode node,
+ IMNode node,
List<Pair<PartialPath, String[]>> timeseriesSchemaList,
boolean needLast,
QueryContext queryContext,
@@ -1393,7 +1500,7 @@ public class MTree implements Serializable {
}
private void addVectorMeasurementSchemaForTemplate(
- MNode node,
+ IMNode node,
List<Pair<PartialPath, String[]>> timeseriesSchemaList,
boolean needLast,
QueryContext queryContext,
@@ -1454,7 +1561,7 @@ public class MTree implements Serializable {
*/
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
private void findChildNodePathInNextLevel(
- MNode node, String[] nodes, int idx, String parent, Set<String> res, int length) {
+ IMNode node, String[] nodes, int idx, String parent, Set<String> res, int length) {
if (node == null) {
return;
}
@@ -1473,7 +1580,7 @@ public class MTree implements Serializable {
}
} else {
if (node.getChildren().size() > 0) {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
if (!Pattern.matches(nodeReg.replace("*", ".*"), child.getName())) {
continue;
}
@@ -1525,7 +1632,7 @@ public class MTree implements Serializable {
*/
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
private void findChildNodeInNextLevel(
- MNode node, String[] nodes, int idx, String parent, Set<String> res, int length) {
+ IMNode node, String[] nodes, int idx, String parent, Set<String> res, int length) {
if (node == null) {
return;
}
@@ -1544,7 +1651,7 @@ public class MTree implements Serializable {
}
} else {
if (node.getChildren().size() > 0) {
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
if (!Pattern.matches(nodeReg.replace("*", ".*"), child.getName())) {
continue;
}
@@ -1615,7 +1722,7 @@ public class MTree implements Serializable {
*/
@SuppressWarnings("squid:S3776")
private void findDevices(
- MNode node,
+ IMNode node,
String[] nodes,
int idx,
Set<PartialPath> res,
@@ -1625,49 +1732,54 @@ public class MTree implements Serializable {
String nodeReg = MetaUtils.getNodeRegByIdx(idx, nodes);
// the node path doesn't contains '*'
if (!nodeReg.contains(PATH_WILDCARD)) {
- MNode next = node.getChild(nodeReg);
+ IMNode next = node.getChild(nodeReg);
if (next != null) {
- if (next instanceof MeasurementMNode && idx >= nodes.length) {
- if (hasLimit) {
- curOffset.set(curOffset.get() + 1);
- if (curOffset.get() < offset.get()
- || count.get().intValue() == limit.get().intValue()) {
- return;
+ if (next instanceof MeasurementMNode) {
+ if (idx >= nodes.length) {
+ if (hasLimit) {
+ curOffset.set(curOffset.get() + 1);
+ if (curOffset.get() < offset.get()
+ || count.get().intValue() == limit.get().intValue()) {
+ return;
+ }
+ count.set(count.get() + 1);
}
- count.set(count.get() + 1);
+ res.add(node.getPartialPath());
}
- res.add(node.getPartialPath());
} else {
findDevices(next, nodes, idx + 1, res, hasLimit, upperTemplate);
}
}
} else { // the node path contains '*'
boolean deviceAdded = false;
- List<MNode> children = new ArrayList<>(node.getChildren().values());
+ List<IMNode> children = new ArrayList<>(node.getChildren().values());
// template part
- if (upperTemplate != null && node.isUseTemplate()) {
+ if (node.isUseTemplate() && upperTemplate != null) {
children.addAll(upperTemplate.getMeasurementMNode());
}
- for (MNode child : children) {
+ for (IMNode child : children) {
// use '.*' to replace '*' to form a regex to match
// if the match failed, skip it.
if (!Pattern.matches(nodeReg.replace("*", ".*"), child.getName())) {
continue;
}
- if (child instanceof MeasurementMNode && !deviceAdded && idx >= nodes.length) {
- if (hasLimit) {
- curOffset.set(curOffset.get() + 1);
- if (curOffset.get() < offset.get()
- || count.get().intValue() == limit.get().intValue()) {
- return;
+ if (child instanceof MeasurementMNode) {
+ if (!deviceAdded && idx >= nodes.length) {
+ if (hasLimit) {
+ curOffset.set(curOffset.get() + 1);
+ if (curOffset.get() < offset.get()
+ || count.get().intValue() == limit.get().intValue()) {
+ return;
+ }
+ count.set(count.get() + 1);
}
- count.set(count.get() + 1);
+ res.add(node.getPartialPath());
+ deviceAdded = true;
}
- res.add(node.getPartialPath());
- deviceAdded = true;
+ } else {
+ findDevices(child, nodes, idx + 1, res, hasLimit, upperTemplate);
}
- findDevices(child, nodes, idx + 1, res, hasLimit, upperTemplate);
}
}
}
@@ -1685,7 +1797,7 @@ public class MTree implements Serializable {
throw new IllegalPathException(path.getFullPath());
}
List<PartialPath> res = new ArrayList<>();
- MNode node = root;
+ IMNode node = root;
for (int i = 1; i < nodes.length; i++) {
if (node.getChild(nodes[i]) != null) {
node = node.getChild(nodes[i]);
@@ -1708,7 +1820,7 @@ public class MTree implements Serializable {
* @param targetLevel Record the distance to the target level, 0 means the target level.
*/
private void findNodes(
- MNode node,
+ IMNode node,
PartialPath path,
List<PartialPath> res,
int targetLevel,
@@ -1723,7 +1835,7 @@ public class MTree implements Serializable {
res.add(path);
return;
}
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
findNodes(child, path.concatNode(child.toString()), res, targetLevel - 1, filter);
}
}
@@ -1736,7 +1848,7 @@ public class MTree implements Serializable {
public static MTree deserializeFrom(File mtreeSnapshot) {
try (MLogReader mLogReader = new MLogReader(mtreeSnapshot)) {
- return deserializeFromReader(mLogReader);
+ return new MTree(deserializeFromReader(mLogReader));
} catch (IOException e) {
logger.warn("Failed to deserialize from {}. Use a new MTree.", mtreeSnapshot.getPath());
return new MTree();
@@ -1749,9 +1861,9 @@ public class MTree implements Serializable {
}
@SuppressWarnings("squid:S3776") // Suppress high Cognitive Complexity warning
- private static MTree deserializeFromReader(MLogReader mLogReader) {
- Deque<MNode> nodeStack = new ArrayDeque<>();
- MNode node = null;
+ private static InternalMNode deserializeFromReader(MLogReader mLogReader) {
+ Deque<IMNode> nodeStack = new ArrayDeque<>();
+ IMNode node = null;
while (mLogReader.hasNext()) {
PhysicalPlan plan = null;
try {
@@ -1767,14 +1879,14 @@ public class MTree implements Serializable {
node = MeasurementMNode.deserializeFrom((MeasurementMNodePlan) plan);
childrenSize = ((MeasurementMNodePlan) plan).getChildSize();
} else if (plan instanceof MNodePlan) {
- node = new MNode(null, ((MNodePlan) plan).getName());
+ node = InternalMNode.deserializeFrom((MNodePlan) plan);
childrenSize = ((MNodePlan) plan).getChildSize();
}
if (childrenSize != 0) {
- ConcurrentHashMap<String, MNode> childrenMap = new ConcurrentHashMap<>();
+ ConcurrentHashMap<String, IMNode> childrenMap = new ConcurrentHashMap<>();
for (int i = 0; i < childrenSize; i++) {
- MNode child = nodeStack.removeFirst();
+ IMNode child = nodeStack.removeFirst();
child.setParent(node);
childrenMap.put(child.getName(), child);
if (child instanceof MeasurementMNode) {
@@ -1792,8 +1904,12 @@ public class MTree implements Serializable {
"Can not operate cmd {} for err:", plan == null ? "" : plan.getOperatorType(), e);
}
}
+ if (!IoTDBConstant.PATH_ROOT.equals(node.getName())) {
+ logger.error("Snapshot file corrupted!");
+ // throw new MetadataException("Snapshot file corrupted!");
+ }
- return new MTree(node);
+ return (InternalMNode) node;
}
@Override
@@ -1803,13 +1919,13 @@ public class MTree implements Serializable {
return jsonToString(jsonObject);
}
- private JsonObject mNodeToJSON(MNode node, String storageGroupName) {
+ private JsonObject mNodeToJSON(IMNode node, String storageGroupName) {
JsonObject jsonObject = new JsonObject();
if (node.getChildren().size() > 0) {
if (node instanceof StorageGroupMNode) {
storageGroupName = node.getFullPath();
}
- for (MNode child : node.getChildren().values()) {
+ for (IMNode child : node.getChildren().values()) {
jsonObject.add(child.getName(), mNodeToJSON(child, storageGroupName));
}
} else if (node instanceof MeasurementMNode) {
@@ -1832,7 +1948,7 @@ public class MTree implements Serializable {
throw new IllegalPathException(path.getFullPath());
}
- Deque<MNode> nodeStack = new ArrayDeque<>();
+ Deque<IMNode> nodeStack = new ArrayDeque<>();
Deque<Integer> depthStack = new ArrayDeque<>();
if (!root.getChildren().isEmpty()) {
nodeStack.push(root);
@@ -1840,10 +1956,10 @@ public class MTree implements Serializable {
}
while (!nodeStack.isEmpty()) {
- MNode mNode = nodeStack.removeFirst();
+ IMNode IMNode = nodeStack.removeFirst();
int depth = depthStack.removeFirst();
- determineStorageGroup(depth + 1, nodes, mNode, paths, nodeStack, depthStack);
+ determineStorageGroup(depth + 1, nodes, IMNode, paths, nodeStack, depthStack);
}
return paths;
}
@@ -1856,17 +1972,17 @@ public class MTree implements Serializable {
private void determineStorageGroup(
int depth,
String[] nodes,
- MNode mNode,
+ IMNode IMNode,
Map<String, String> paths,
- Deque<MNode> nodeStack,
+ Deque<IMNode> nodeStack,
Deque<Integer> depthStack) {
String currNode = depth >= nodes.length ? PATH_WILDCARD : nodes[depth];
- for (Entry<String, MNode> entry : mNode.getChildren().entrySet()) {
+ for (Entry<String, IMNode> entry : IMNode.getChildren().entrySet()) {
if (!currNode.equals(PATH_WILDCARD) && !currNode.equals(entry.getKey())) {
continue;
}
// this child is desired
- MNode child = entry.getValue();
+ IMNode child = entry.getValue();
if (child instanceof StorageGroupMNode) {
// we have found one storage group, record it
String sgName = child.getFullPath();
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/MetaUtils.java b/server/src/main/java/org/apache/iotdb/db/metadata/MetaUtils.java
index 35c80bc..84672d8 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/MetaUtils.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/MetaUtils.java
@@ -21,7 +21,7 @@ package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.conf.IoTDBConstant;
import org.apache.iotdb.db.exception.metadata.IllegalPathException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.utils.TestOnly;
import java.util.ArrayList;
@@ -118,32 +118,32 @@ public class MetaUtils {
}
@TestOnly
- public static List<String> getMultiFullPaths(MNode node) {
+ public static List<String> getMultiFullPaths(IMNode node) {
if (node == null) {
return Collections.emptyList();
}
- List<MNode> lastNodeList = new ArrayList<>();
+ List<IMNode> lastNodeList = new ArrayList<>();
collectLastNode(node, lastNodeList);
List<String> result = new ArrayList<>();
- for (MNode mNode : lastNodeList) {
- result.add(mNode.getFullPath());
+ for (IMNode IMNode : lastNodeList) {
+ result.add(IMNode.getFullPath());
}
return result;
}
@TestOnly
- public static void collectLastNode(MNode node, List<MNode> lastNodeList) {
+ public static void collectLastNode(IMNode node, List<IMNode> lastNodeList) {
if (node != null) {
- Map<String, MNode> children = node.getChildren();
+ Map<String, IMNode> children = node.getChildren();
if (children.isEmpty()) {
lastNodeList.add(node);
}
- for (Entry<String, MNode> entry : children.entrySet()) {
- MNode childNode = entry.getValue();
+ for (Entry<String, IMNode> entry : children.entrySet()) {
+ IMNode childNode = entry.getValue();
collectLastNode(childNode, lastNodeList);
}
}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/logfile/MLogWriter.java b/server/src/main/java/org/apache/iotdb/db/metadata/logfile/MLogWriter.java
index b18f9a1..4448399 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/logfile/MLogWriter.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/logfile/MLogWriter.java
@@ -24,7 +24,7 @@ import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.metadata.MetadataConstant;
import org.apache.iotdb.db.metadata.MetadataOperationType;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.metadata.mnode.StorageGroupMNode;
import org.apache.iotdb.db.qp.physical.PhysicalPlan;
@@ -185,7 +185,7 @@ public class MLogWriter implements AutoCloseable {
putLog(plan);
}
- public void serializeMNode(MNode node) throws IOException {
+ public void serializeMNode(IMNode node) throws IOException {
int childSize = 0;
if (node.getChildren() != null) {
childSize = node.getChildren().size();
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/IMNode.java b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/IMNode.java
new file mode 100644
index 0000000..4b874b2
--- /dev/null
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/IMNode.java
@@ -0,0 +1,86 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.iotdb.db.metadata.mnode;
+
+import org.apache.iotdb.db.exception.metadata.MetadataException;
+import org.apache.iotdb.db.metadata.PartialPath;
+import org.apache.iotdb.db.metadata.logfile.MLogWriter;
+import org.apache.iotdb.db.metadata.template.Template;
+
+import java.io.IOException;
+import java.io.Serializable;
+import java.util.Map;
+
+/** This interface defines a MNode's operation interfaces. */
+public interface IMNode extends Serializable {
+
+ String getName();
+
+ void setName(String name);
+
+ IMNode getParent();
+
+ void setParent(IMNode parent);
+
+ String getFullPath();
+
+ void setFullPath(String fullPath);
+
+ PartialPath getPartialPath();
+
+ boolean hasChild(String name);
+
+ IMNode getChild(String name);
+
+ void addChild(String name, IMNode child);
+
+ IMNode addChild(IMNode child);
+
+ boolean addAlias(String alias, IMNode child);
+
+ void deleteChild(String name);
+
+ void deleteAliasChild(String alias);
+
+ void replaceChild(String measurement, IMNode newChildNode);
+
+ IMNode getChildOfAlignedTimeseries(String name) throws MetadataException;
+
+ Map<String, IMNode> getChildren();
+
+ Map<String, IMNode> getAliasChildren();
+
+ void setChildren(Map<String, IMNode> children);
+
+ void setAliasChildren(Map<String, IMNode> aliasChildren);
+
+ boolean isUseTemplate();
+
+ void setUseTemplate(boolean useTemplate);
+
+ Template getUpperTemplate();
+
+ Template getDeviceTemplate();
+
+ void setDeviceTemplate(Template deviceTemplate);
+
+ int getMeasurementMNodeCount();
+
+ void serializeTo(MLogWriter logWriter) throws IOException;
+}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/InternalMNode.java
similarity index 65%
copy from server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java
copy to server/src/main/java/org/apache/iotdb/db/metadata/mnode/InternalMNode.java
index 3296cc3..89a6ba0 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/InternalMNode.java
@@ -18,42 +18,28 @@
*/
package org.apache.iotdb.db.metadata.mnode;
-import org.apache.iotdb.db.conf.IoTDBConstant;
import org.apache.iotdb.db.exception.metadata.AlignedTimeseriesException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.metadata.MetaUtils;
import org.apache.iotdb.db.metadata.PartialPath;
import org.apache.iotdb.db.metadata.logfile.MLogWriter;
import org.apache.iotdb.db.metadata.template.Template;
-import org.apache.iotdb.db.rescon.CachedStringPool;
+import org.apache.iotdb.db.qp.physical.sys.MNodePlan;
import java.io.IOException;
-import java.io.Serializable;
-import java.util.ArrayList;
import java.util.Collections;
import java.util.List;
import java.util.Map;
import java.util.Map.Entry;
-import java.util.Objects;
import java.util.concurrent.ConcurrentHashMap;
/**
* This class is the implementation of Metadata Node. One MNode instance represents one node in the
* Metadata Tree
*/
-public class MNode implements Serializable {
+public class InternalMNode extends MNode implements IMNode {
private static final long serialVersionUID = -770028375899514063L;
- private static Map<String, String> cachedPathPool =
- CachedStringPool.getInstance().getCachedPool();
-
- /** Name of the MNode */
- protected String name;
-
- protected MNode parent;
-
- /** from root to this node, only be set when used once for InternalMNode */
- protected String fullPath;
/**
* use in Measurement Node so it's protected suppress warnings reason: volatile for double
@@ -62,7 +48,7 @@ public class MNode implements Serializable {
* <p>This will be a ConcurrentHashMap instance
*/
@SuppressWarnings("squid:S3077")
- protected transient volatile Map<String, MNode> children = null;
+ protected transient volatile Map<String, IMNode> children = null;
/**
* suppress warnings reason: volatile for double synchronized check
@@ -70,7 +56,7 @@ public class MNode implements Serializable {
* <p>This will be a ConcurrentHashMap instance
*/
@SuppressWarnings("squid:S3077")
- private transient volatile Map<String, MNode> aliasChildren = null;
+ private transient volatile Map<String, IMNode> aliasChildren = null;
// device template
protected Template deviceTemplate = null;
@@ -78,12 +64,12 @@ public class MNode implements Serializable {
private volatile boolean useTemplate = false;
/** Constructor of MNode. */
- public MNode(MNode parent, String name) {
- this.parent = parent;
- this.name = name;
+ public InternalMNode(IMNode parent, String name) {
+ super(parent, name);
}
/** check whether the MNode has a child with the name */
+ @Override
public boolean hasChild(String name) {
return (children != null && children.containsKey(name))
|| (aliasChildren != null && aliasChildren.containsKey(name));
@@ -95,7 +81,8 @@ public class MNode implements Serializable {
* @param name child's name
* @param child child's node
*/
- public void addChild(String name, MNode child) {
+ @Override
+ public void addChild(String name, IMNode child) {
/* use cpu time to exchange memory
* measurementNode's children should be null to save memory
* add child method will only be called when writing MTree, which is not a frequent operation
@@ -108,7 +95,7 @@ public class MNode implements Serializable {
}
}
}
- child.parent = this;
+ child.setParent(this);
children.putIfAbsent(name, child);
}
@@ -123,7 +110,7 @@ public class MNode implements Serializable {
* @param child child's node
* @return return the MNode already added
*/
- MNode addChild(MNode child) {
+ public IMNode addChild(IMNode child) {
/* use cpu time to exchange memory
* measurementNode's children should be null to save memory
* add child method will only be called when writing MTree, which is not a frequent operation
@@ -137,12 +124,13 @@ public class MNode implements Serializable {
}
}
- child.parent = this;
+ child.setParent(this);
children.putIfAbsent(child.getName(), child);
return child;
}
/** delete a child */
+ @Override
public void deleteChild(String name) {
if (children != null) {
children.remove(name);
@@ -150,23 +138,27 @@ public class MNode implements Serializable {
}
/** delete the alias of a child */
+ @Override
public void deleteAliasChild(String alias) {
if (aliasChildren != null) {
aliasChildren.remove(alias);
}
}
+ @Override
public Template getDeviceTemplate() {
return deviceTemplate;
}
+ @Override
public void setDeviceTemplate(Template deviceTemplate) {
this.deviceTemplate = deviceTemplate;
}
/** get the child with the name */
- public MNode getChild(String name) {
- MNode child = null;
+ @Override
+ public IMNode getChild(String name) {
+ IMNode child = null;
if (children != null) {
child = children.get(name);
}
@@ -176,12 +168,13 @@ public class MNode implements Serializable {
return aliasChildren == null ? null : aliasChildren.get(name);
}
- public MNode getChildOfAlignedTimeseries(String name) throws MetadataException {
- MNode node = null;
+ @Override
+ public IMNode getChildOfAlignedTimeseries(String name) throws MetadataException {
+ IMNode node = null;
// for aligned timeseries
List<String> measurementList = MetaUtils.getMeasurementsInPartialPath(new PartialPath(name));
for (String measurement : measurementList) {
- MNode nodeOfMeasurement = getChild(measurement);
+ IMNode nodeOfMeasurement = getChild(measurement);
if (node == null) {
node = nodeOfMeasurement;
} else {
@@ -195,22 +188,21 @@ public class MNode implements Serializable {
}
/** get the count of all MeasurementMNode whose ancestor is current node */
+ @Override
public int getMeasurementMNodeCount() {
if (children == null) {
- return 1;
+ return 0;
}
int measurementMNodeCount = 0;
- if (this instanceof MeasurementMNode) {
- measurementMNodeCount += 1; // current node itself may be MeasurementMNode
- }
- for (MNode child : children.values()) {
+ for (IMNode child : children.values()) {
measurementMNodeCount += child.getMeasurementMNodeCount();
}
return measurementMNodeCount;
}
/** add an alias */
- public boolean addAlias(String alias, MNode child) {
+ @Override
+ public boolean addAlias(String alias, IMNode child) {
if (aliasChildren == null) {
// double check, alias children volatile
synchronized (this) {
@@ -223,89 +215,32 @@ public class MNode implements Serializable {
return aliasChildren.computeIfAbsent(alias, aliasName -> child) == child;
}
- /** get full path */
- public String getFullPath() {
- if (fullPath == null) {
- fullPath = concatFullPath();
- String cachedFullPath = cachedPathPool.get(fullPath);
- if (cachedFullPath == null) {
- cachedPathPool.put(fullPath, fullPath);
- } else {
- fullPath = cachedFullPath;
- }
- }
- return fullPath;
- }
-
- /**
- * get partial path of this node
- *
- * @return partial path
- */
- public PartialPath getPartialPath() {
- List<String> detachedPath = new ArrayList<>();
- MNode temp = this;
- detachedPath.add(temp.getName());
- while (temp.getParent() != null) {
- temp = temp.getParent();
- detachedPath.add(0, temp.getName());
- }
- return new PartialPath(detachedPath.toArray(new String[0]));
- }
-
- String concatFullPath() {
- StringBuilder builder = new StringBuilder(name);
- MNode curr = this;
- while (curr.getParent() != null) {
- curr = curr.getParent();
- builder.insert(0, IoTDBConstant.PATH_SEPARATOR).insert(0, curr.name);
- }
- return builder.toString();
- }
-
@Override
- public String toString() {
- return this.getName();
- }
-
- public MNode getParent() {
- return parent;
- }
-
- public void setParent(MNode parent) {
- this.parent = parent;
- }
-
- public Map<String, MNode> getChildren() {
+ public Map<String, IMNode> getChildren() {
if (children == null) {
return Collections.emptyMap();
}
return children;
}
- public Map<String, MNode> getAliasChildren() {
+ @Override
+ public Map<String, IMNode> getAliasChildren() {
if (aliasChildren == null) {
return Collections.emptyMap();
}
return aliasChildren;
}
- public void setChildren(Map<String, MNode> children) {
+ @Override
+ public void setChildren(Map<String, IMNode> children) {
this.children = children;
}
- private void setAliasChildren(Map<String, MNode> aliasChildren) {
+ public void setAliasChildren(Map<String, IMNode> aliasChildren) {
this.aliasChildren = aliasChildren;
}
- public String getName() {
- return name;
- }
-
- public void setName(String name) {
- this.name = name;
- }
-
+ @Override
public void serializeTo(MLogWriter logWriter) throws IOException {
serializeChildren(logWriter);
@@ -316,30 +251,35 @@ public class MNode implements Serializable {
if (children == null) {
return;
}
- for (Entry<String, MNode> entry : children.entrySet()) {
+ for (Entry<String, IMNode> entry : children.entrySet()) {
entry.getValue().serializeTo(logWriter);
}
}
+ public static InternalMNode deserializeFrom(MNodePlan plan) {
+ return new InternalMNode(null, plan.getName());
+ }
+
/**
* replace a child of this mnode
*
* @param measurement measurement name
* @param newChildNode new child node
*/
- public void replaceChild(String measurement, MNode newChildNode) {
- MNode oldChildNode = this.getChild(measurement);
+ @Override
+ public void replaceChild(String measurement, IMNode newChildNode) {
+ IMNode oldChildNode = this.getChild(measurement);
if (oldChildNode == null) {
return;
}
// newChildNode builds parent-child relationship
- Map<String, MNode> grandChildren = oldChildNode.getChildren();
+ Map<String, IMNode> grandChildren = oldChildNode.getChildren();
newChildNode.setChildren(grandChildren);
grandChildren.forEach(
(grandChildName, grandChildNode) -> grandChildNode.setParent(newChildNode));
- Map<String, MNode> grandAliasChildren = oldChildNode.getAliasChildren();
+ Map<String, IMNode> grandAliasChildren = oldChildNode.getAliasChildren();
newChildNode.setAliasChildren(grandAliasChildren);
grandAliasChildren.forEach(
(grandAliasChildName, grandAliasChild) -> grandAliasChild.setParent(newChildNode));
@@ -350,56 +290,30 @@ public class MNode implements Serializable {
this.addChild(newChildNode.getName(), newChildNode);
}
- public void setFullPath(String fullPath) {
- this.fullPath = fullPath;
- }
-
/**
* get upper template of this node, remember we get nearest template alone this node to root
*
* @return upper template
*/
+ @Override
public Template getUpperTemplate() {
- MNode cur = this;
+ IMNode cur = this;
while (cur != null) {
if (cur.getDeviceTemplate() != null) {
- return cur.deviceTemplate;
+ return cur.getDeviceTemplate();
}
- cur = cur.parent;
+ cur = cur.getParent();
}
return null;
}
@Override
- public boolean equals(Object o) {
- if (this == o) {
- return true;
- }
- if (o == null || getClass() != o.getClass()) {
- return false;
- }
- MNode mNode = (MNode) o;
- if (fullPath == null) {
- return Objects.equals(getFullPath(), mNode.getFullPath());
- } else {
- return Objects.equals(fullPath, mNode.fullPath);
- }
- }
-
- @Override
- public int hashCode() {
- if (fullPath == null) {
- return Objects.hash(getFullPath());
- } else {
- return Objects.hash(fullPath);
- }
- }
-
public boolean isUseTemplate() {
return useTemplate;
}
+ @Override
public void setUseTemplate(boolean useTemplate) {
this.useTemplate = useTemplate;
}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java
index 3296cc3..4cd7df1 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MNode.java
@@ -19,211 +19,72 @@
package org.apache.iotdb.db.metadata.mnode;
import org.apache.iotdb.db.conf.IoTDBConstant;
-import org.apache.iotdb.db.exception.metadata.AlignedTimeseriesException;
-import org.apache.iotdb.db.exception.metadata.MetadataException;
-import org.apache.iotdb.db.metadata.MetaUtils;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.logfile.MLogWriter;
-import org.apache.iotdb.db.metadata.template.Template;
import org.apache.iotdb.db.rescon.CachedStringPool;
-import java.io.IOException;
-import java.io.Serializable;
import java.util.ArrayList;
-import java.util.Collections;
import java.util.List;
import java.util.Map;
-import java.util.Map.Entry;
import java.util.Objects;
-import java.util.concurrent.ConcurrentHashMap;
-/**
- * This class is the implementation of Metadata Node. One MNode instance represents one node in the
- * Metadata Tree
- */
-public class MNode implements Serializable {
+public abstract class MNode implements IMNode {
- private static final long serialVersionUID = -770028375899514063L;
private static Map<String, String> cachedPathPool =
CachedStringPool.getInstance().getCachedPool();
/** Name of the MNode */
protected String name;
- protected MNode parent;
+ protected IMNode parent;
/** from root to this node, only be set when used once for InternalMNode */
protected String fullPath;
- /**
- * use in Measurement Node so it's protected suppress warnings reason: volatile for double
- * synchronized check
- *
- * <p>This will be a ConcurrentHashMap instance
- */
- @SuppressWarnings("squid:S3077")
- protected transient volatile Map<String, MNode> children = null;
-
- /**
- * suppress warnings reason: volatile for double synchronized check
- *
- * <p>This will be a ConcurrentHashMap instance
- */
- @SuppressWarnings("squid:S3077")
- private transient volatile Map<String, MNode> aliasChildren = null;
-
- // device template
- protected Template deviceTemplate = null;
-
- private volatile boolean useTemplate = false;
-
/** Constructor of MNode. */
- public MNode(MNode parent, String name) {
+ public MNode(IMNode parent, String name) {
this.parent = parent;
this.name = name;
}
- /** check whether the MNode has a child with the name */
- public boolean hasChild(String name) {
- return (children != null && children.containsKey(name))
- || (aliasChildren != null && aliasChildren.containsKey(name));
- }
-
- /**
- * add a child to current mnode
- *
- * @param name child's name
- * @param child child's node
- */
- public void addChild(String name, MNode child) {
- /* use cpu time to exchange memory
- * measurementNode's children should be null to save memory
- * add child method will only be called when writing MTree, which is not a frequent operation
- */
- if (children == null) {
- // double check, children is volatile
- synchronized (this) {
- if (children == null) {
- children = new ConcurrentHashMap<>();
- }
- }
- }
- child.parent = this;
- children.putIfAbsent(name, child);
- }
-
- /**
- * Add a child to the current mnode.
- *
- * <p>This method will not take the child's name as one of the inputs and will also make this
- * Mnode be child node's parent. All is to reduce the probability of mistaken by users and be more
- * convenient for users to use. And the return of this method is used to conveniently construct a
- * chain of time series for users.
- *
- * @param child child's node
- * @return return the MNode already added
- */
- MNode addChild(MNode child) {
- /* use cpu time to exchange memory
- * measurementNode's children should be null to save memory
- * add child method will only be called when writing MTree, which is not a frequent operation
- */
- if (children == null) {
- // double check, children is volatile
- synchronized (this) {
- if (children == null) {
- children = new ConcurrentHashMap<>();
- }
- }
- }
-
- child.parent = this;
- children.putIfAbsent(child.getName(), child);
- return child;
- }
-
- /** delete a child */
- public void deleteChild(String name) {
- if (children != null) {
- children.remove(name);
- }
- }
-
- /** delete the alias of a child */
- public void deleteAliasChild(String alias) {
- if (aliasChildren != null) {
- aliasChildren.remove(alias);
- }
- }
-
- public Template getDeviceTemplate() {
- return deviceTemplate;
- }
-
- public void setDeviceTemplate(Template deviceTemplate) {
- this.deviceTemplate = deviceTemplate;
+ @Override
+ public String getName() {
+ return name;
}
- /** get the child with the name */
- public MNode getChild(String name) {
- MNode child = null;
- if (children != null) {
- child = children.get(name);
- }
- if (child != null) {
- return child;
- }
- return aliasChildren == null ? null : aliasChildren.get(name);
+ @Override
+ public void setName(String name) {
+ this.name = name;
}
- public MNode getChildOfAlignedTimeseries(String name) throws MetadataException {
- MNode node = null;
- // for aligned timeseries
- List<String> measurementList = MetaUtils.getMeasurementsInPartialPath(new PartialPath(name));
- for (String measurement : measurementList) {
- MNode nodeOfMeasurement = getChild(measurement);
- if (node == null) {
- node = nodeOfMeasurement;
- } else {
- if (node != nodeOfMeasurement) {
- throw new AlignedTimeseriesException(
- "Cannot get node of children in different aligned timeseries", name);
- }
- }
- }
- return node;
+ @Override
+ public IMNode getParent() {
+ return parent;
}
- /** get the count of all MeasurementMNode whose ancestor is current node */
- public int getMeasurementMNodeCount() {
- if (children == null) {
- return 1;
- }
- int measurementMNodeCount = 0;
- if (this instanceof MeasurementMNode) {
- measurementMNodeCount += 1; // current node itself may be MeasurementMNode
- }
- for (MNode child : children.values()) {
- measurementMNodeCount += child.getMeasurementMNodeCount();
- }
- return measurementMNodeCount;
+ @Override
+ public void setParent(IMNode parent) {
+ this.parent = parent;
}
- /** add an alias */
- public boolean addAlias(String alias, MNode child) {
- if (aliasChildren == null) {
- // double check, alias children volatile
- synchronized (this) {
- if (aliasChildren == null) {
- aliasChildren = new ConcurrentHashMap<>();
- }
- }
+ /**
+ * get partial path of this node
+ *
+ * @return partial path
+ */
+ @Override
+ public PartialPath getPartialPath() {
+ List<String> detachedPath = new ArrayList<>();
+ IMNode temp = this;
+ detachedPath.add(temp.getName());
+ while (temp.getParent() != null) {
+ temp = temp.getParent();
+ detachedPath.add(0, temp.getName());
}
-
- return aliasChildren.computeIfAbsent(alias, aliasName -> child) == child;
+ return new PartialPath(detachedPath.toArray(new String[0]));
}
/** get full path */
+ @Override
public String getFullPath() {
if (fullPath == null) {
fullPath = concatFullPath();
@@ -237,140 +98,21 @@ public class MNode implements Serializable {
return fullPath;
}
- /**
- * get partial path of this node
- *
- * @return partial path
- */
- public PartialPath getPartialPath() {
- List<String> detachedPath = new ArrayList<>();
- MNode temp = this;
- detachedPath.add(temp.getName());
- while (temp.getParent() != null) {
- temp = temp.getParent();
- detachedPath.add(0, temp.getName());
- }
- return new PartialPath(detachedPath.toArray(new String[0]));
- }
-
String concatFullPath() {
StringBuilder builder = new StringBuilder(name);
- MNode curr = this;
+ IMNode curr = this;
while (curr.getParent() != null) {
curr = curr.getParent();
- builder.insert(0, IoTDBConstant.PATH_SEPARATOR).insert(0, curr.name);
+ builder.insert(0, IoTDBConstant.PATH_SEPARATOR).insert(0, curr.getName());
}
return builder.toString();
}
@Override
- public String toString() {
- return this.getName();
- }
-
- public MNode getParent() {
- return parent;
- }
-
- public void setParent(MNode parent) {
- this.parent = parent;
- }
-
- public Map<String, MNode> getChildren() {
- if (children == null) {
- return Collections.emptyMap();
- }
- return children;
- }
-
- public Map<String, MNode> getAliasChildren() {
- if (aliasChildren == null) {
- return Collections.emptyMap();
- }
- return aliasChildren;
- }
-
- public void setChildren(Map<String, MNode> children) {
- this.children = children;
- }
-
- private void setAliasChildren(Map<String, MNode> aliasChildren) {
- this.aliasChildren = aliasChildren;
- }
-
- public String getName() {
- return name;
- }
-
- public void setName(String name) {
- this.name = name;
- }
-
- public void serializeTo(MLogWriter logWriter) throws IOException {
- serializeChildren(logWriter);
-
- logWriter.serializeMNode(this);
- }
-
- void serializeChildren(MLogWriter logWriter) throws IOException {
- if (children == null) {
- return;
- }
- for (Entry<String, MNode> entry : children.entrySet()) {
- entry.getValue().serializeTo(logWriter);
- }
- }
-
- /**
- * replace a child of this mnode
- *
- * @param measurement measurement name
- * @param newChildNode new child node
- */
- public void replaceChild(String measurement, MNode newChildNode) {
- MNode oldChildNode = this.getChild(measurement);
- if (oldChildNode == null) {
- return;
- }
-
- // newChildNode builds parent-child relationship
- Map<String, MNode> grandChildren = oldChildNode.getChildren();
- newChildNode.setChildren(grandChildren);
- grandChildren.forEach(
- (grandChildName, grandChildNode) -> grandChildNode.setParent(newChildNode));
-
- Map<String, MNode> grandAliasChildren = oldChildNode.getAliasChildren();
- newChildNode.setAliasChildren(grandAliasChildren);
- grandAliasChildren.forEach(
- (grandAliasChildName, grandAliasChild) -> grandAliasChild.setParent(newChildNode));
-
- newChildNode.setParent(this);
-
- this.deleteChild(measurement);
- this.addChild(newChildNode.getName(), newChildNode);
- }
-
public void setFullPath(String fullPath) {
this.fullPath = fullPath;
}
- /**
- * get upper template of this node, remember we get nearest template alone this node to root
- *
- * @return upper template
- */
- public Template getUpperTemplate() {
- MNode cur = this;
- while (cur != null) {
- if (cur.getDeviceTemplate() != null) {
- return cur.deviceTemplate;
- }
- cur = cur.parent;
- }
-
- return null;
- }
-
@Override
public boolean equals(Object o) {
if (this == o) {
@@ -396,11 +138,8 @@ public class MNode implements Serializable {
}
}
- public boolean isUseTemplate() {
- return useTemplate;
- }
-
- public void setUseTemplate(boolean useTemplate) {
- this.useTemplate = useTemplate;
+ @Override
+ public String toString() {
+ return this.getName();
}
}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MeasurementMNode.java b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MeasurementMNode.java
index e2d309b..901e1b9 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MeasurementMNode.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/MeasurementMNode.java
@@ -19,7 +19,9 @@
package org.apache.iotdb.db.metadata.mnode;
import org.apache.iotdb.db.engine.trigger.executor.TriggerExecutor;
+import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.metadata.logfile.MLogWriter;
+import org.apache.iotdb.db.metadata.template.Template;
import org.apache.iotdb.db.qp.physical.sys.MeasurementMNodePlan;
import org.apache.iotdb.tsfile.file.metadata.enums.CompressionType;
import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
@@ -28,13 +30,19 @@ import org.apache.iotdb.tsfile.read.TimeValuePair;
import org.apache.iotdb.tsfile.write.schema.IMeasurementSchema;
import org.apache.iotdb.tsfile.write.schema.MeasurementSchema;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
import java.io.IOException;
+import java.util.Collections;
import java.util.HashMap;
import java.util.Map;
/** Represents an MNode which has a Measurement or Sensor attached to it. */
public class MeasurementMNode extends MNode {
+ private static final Logger logger = LoggerFactory.getLogger(MeasurementMNode.class);
+
private static final long serialVersionUID = -1199657856921206435L;
/** measurement's Schema for one timeseries represented by current leaf node */
@@ -54,7 +62,7 @@ public class MeasurementMNode extends MNode {
/** @param alias alias of measurementName */
public MeasurementMNode(
- MNode parent,
+ IMNode parent,
String measurementName,
String alias,
TSDataType dataType,
@@ -67,12 +75,21 @@ public class MeasurementMNode extends MNode {
}
public MeasurementMNode(
- MNode parent, String measurementName, IMeasurementSchema schema, String alias) {
+ IMNode parent, String measurementName, IMeasurementSchema schema, String alias) {
super(parent, measurementName);
this.schema = schema;
this.alias = alias;
}
+ @Override
+ public int getMeasurementMNodeCount() {
+ return 1;
+ }
+
+ public int getMeasurementCount() {
+ return schema.getMeasurementCount();
+ }
+
public IMeasurementSchema getSchema() {
return schema;
}
@@ -148,8 +165,6 @@ public class MeasurementMNode extends MNode {
@Override
public void serializeTo(MLogWriter logWriter) throws IOException {
- serializeChildren(logWriter);
-
logWriter.serializeMeasurementMNode(this);
}
@@ -205,4 +220,90 @@ public class MeasurementMNode extends MNode {
return schema.getValueTSDataTypeList().get(index);
}
}
+
+ @Override
+ public boolean hasChild(String name) {
+ return false;
+ }
+
+ @Override
+ public void addChild(String name, IMNode child) {
+ // Do nothing
+ }
+
+ @Override
+ public IMNode addChild(IMNode child) {
+ return null;
+ }
+
+ @Override
+ public void deleteChild(String name) {
+ // Do nothing
+ }
+
+ @Override
+ public void deleteAliasChild(String alias) {
+ // Do nothing
+ }
+
+ @Override
+ public IMNode getChild(String name) {
+ logger.warn("current node {} is a MeasurementMNode, can not get child {}", super.name, name);
+ throw new RuntimeException(
+ String.format(
+ "current node %s is a MeasurementMNode, can not get child %s", super.name, name));
+ }
+
+ @Override
+ public IMNode getChildOfAlignedTimeseries(String name) throws MetadataException {
+ return null;
+ }
+
+ @Override
+ public boolean addAlias(String alias, IMNode child) {
+ return false;
+ }
+
+ @Override
+ public Map<String, IMNode> getChildren() {
+ return Collections.emptyMap();
+ }
+
+ @Override
+ public Map<String, IMNode> getAliasChildren() {
+ return null;
+ }
+
+ public void setChildren(Map<String, IMNode> children) {
+ // Do nothing
+ }
+
+ @Override
+ public void setAliasChildren(Map<String, IMNode> aliasChildren) {}
+
+ @Override
+ public void replaceChild(String measurement, IMNode newChildNode) {}
+
+ @Override
+ public Template getUpperTemplate() {
+ return parent.getUpperTemplate();
+ }
+
+ @Override
+ public boolean isUseTemplate() {
+ return false;
+ }
+
+ @Override
+ public Template getDeviceTemplate() {
+ logger.warn("current node {} is a MeasurementMNode, can not get Device Template", name);
+ throw new RuntimeException(
+ String.format("current node %s is a MeasurementMNode, can not get Device Template", name));
+ }
+
+ @Override
+ public void setDeviceTemplate(Template deviceTemplate) {}
+
+ @Override
+ public void setUseTemplate(boolean useTemplate) {}
}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/StorageGroupMNode.java b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/StorageGroupMNode.java
index 8453fcf..b229427 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/mnode/StorageGroupMNode.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/mnode/StorageGroupMNode.java
@@ -23,7 +23,7 @@ import org.apache.iotdb.db.qp.physical.sys.StorageGroupMNodePlan;
import java.io.IOException;
-public class StorageGroupMNode extends MNode {
+public class StorageGroupMNode extends InternalMNode {
private static final long serialVersionUID = 7999036474525817732L;
@@ -33,7 +33,7 @@ public class StorageGroupMNode extends MNode {
*/
private long dataTTL;
- public StorageGroupMNode(MNode parent, String name, long dataTTL) {
+ public StorageGroupMNode(IMNode parent, String name, long dataTTL) {
super(parent, name);
this.dataTTL = dataTTL;
}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/tag/TagManager.java b/server/src/main/java/org/apache/iotdb/db/metadata/tag/TagManager.java
new file mode 100644
index 0000000..b621930
--- /dev/null
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/tag/TagManager.java
@@ -0,0 +1,555 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.iotdb.db.metadata.tag;
+
+import org.apache.iotdb.db.conf.IoTDBConfig;
+import org.apache.iotdb.db.conf.IoTDBDescriptor;
+import org.apache.iotdb.db.engine.StorageEngine;
+import org.apache.iotdb.db.engine.storagegroup.StorageGroupProcessor;
+import org.apache.iotdb.db.exception.StorageEngineException;
+import org.apache.iotdb.db.exception.metadata.MetadataException;
+import org.apache.iotdb.db.metadata.MTree;
+import org.apache.iotdb.db.metadata.MetadataConstant;
+import org.apache.iotdb.db.metadata.PartialPath;
+import org.apache.iotdb.db.metadata.logfile.TagLogFile;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
+import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
+import org.apache.iotdb.db.qp.physical.sys.ShowTimeSeriesPlan;
+import org.apache.iotdb.db.query.context.QueryContext;
+import org.apache.iotdb.db.utils.TestOnly;
+import org.apache.iotdb.tsfile.utils.Pair;
+
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.util.*;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.CopyOnWriteArraySet;
+
+import static java.util.stream.Collectors.toList;
+
+public class TagManager {
+
+ private static final String TAG_FORMAT = "tag key is %s, tag value is %s, tlog offset is %d";
+ private static final String DEBUG_MSG = "%s : TimeSeries %s is removed from tag inverted index, ";
+ private static final String DEBUG_MSG_1 =
+ "%s: TimeSeries %s's tag info has been removed from tag inverted index ";
+ private static final String PREVIOUS_CONDITION =
+ "before deleting it, tag key is %s, tag value is %s, tlog offset is %d, contains key %b";
+
+ private static final Logger logger = LoggerFactory.getLogger(TagManager.class);
+ private static IoTDBConfig config = IoTDBDescriptor.getInstance().getConfig();
+
+ private TagLogFile tagLogFile;
+ // tag key -> tag value -> LeafMNode
+ private Map<String, Map<String, Set<MeasurementMNode>>> tagIndex = new ConcurrentHashMap<>();
+
+ private static class TagManagerHolder {
+
+ private TagManagerHolder() {
+ // allowed to do nothing
+ }
+
+ private static final TagManager INSTANCE = new TagManager();
+ }
+
+ public static TagManager getInstance() {
+ return TagManagerHolder.INSTANCE;
+ }
+
+ @TestOnly
+ public static TagManager getNewInstanceForTest() {
+ return new TagManager();
+ }
+
+ private TagManager() {}
+
+ public void init() throws IOException {
+ tagLogFile = new TagLogFile(config.getSchemaDir(), MetadataConstant.TAG_LOG);
+ }
+
+ public void addIndex(String tagKey, String tagValue, MeasurementMNode measurementMNode) {
+ tagIndex
+ .computeIfAbsent(tagKey, k -> new ConcurrentHashMap<>())
+ .computeIfAbsent(tagValue, v -> new CopyOnWriteArraySet<>())
+ .add(measurementMNode);
+ }
+
+ public void addIndex(Map<String, String> tagsMap, MeasurementMNode measurementMNode) {
+ if (tagsMap != null && measurementMNode != null) {
+ for (Map.Entry<String, String> entry : tagsMap.entrySet()) {
+ addIndex(entry.getKey(), entry.getValue(), measurementMNode);
+ }
+ }
+ }
+
+ public void removeIndex(String tagKey, String tagValue, MeasurementMNode measurementMNode) {
+ tagIndex.get(tagKey).get(tagValue).remove(measurementMNode);
+ if (tagIndex.get(tagKey).get(tagValue).isEmpty()) {
+ tagIndex.get(tagKey).remove(tagValue);
+ }
+ }
+
+ public List<MeasurementMNode> getMatchedTimeseriesInIndex(
+ ShowTimeSeriesPlan plan, QueryContext context) throws MetadataException {
+ if (!tagIndex.containsKey(plan.getKey())) {
+ throw new MetadataException("The key " + plan.getKey() + " is not a tag.", true);
+ }
+ Map<String, Set<MeasurementMNode>> value2Node = tagIndex.get(plan.getKey());
+ if (value2Node.isEmpty()) {
+ throw new MetadataException("The key " + plan.getKey() + " is not a tag.");
+ }
+
+ List<MeasurementMNode> allMatchedNodes = new ArrayList<>();
+ if (plan.isContains()) {
+ for (Map.Entry<String, Set<MeasurementMNode>> entry : value2Node.entrySet()) {
+ if (entry.getKey() == null || entry.getValue() == null) {
+ continue;
+ }
+ String tagValue = entry.getKey();
+ if (tagValue.contains(plan.getValue())) {
+ allMatchedNodes.addAll(entry.getValue());
+ }
+ }
+ } else {
+ for (Map.Entry<String, Set<MeasurementMNode>> entry : value2Node.entrySet()) {
+ if (entry.getKey() == null || entry.getValue() == null) {
+ continue;
+ }
+ String tagValue = entry.getKey();
+ if (plan.getValue().equals(tagValue)) {
+ allMatchedNodes.addAll(entry.getValue());
+ }
+ }
+ }
+
+ // if ordered by heat, we sort all the timeseries by the descending order of the last insert
+ // timestamp
+ if (plan.isOrderByHeat()) {
+ List<StorageGroupProcessor> list;
+ try {
+ list =
+ StorageEngine.getInstance()
+ .mergeLock(allMatchedNodes.stream().map(IMNode::getPartialPath).collect(toList()));
+ try {
+ allMatchedNodes =
+ allMatchedNodes.stream()
+ .sorted(
+ Comparator.comparingLong(
+ (MeasurementMNode mNode) -> MTree.getLastTimeStamp(mNode, context))
+ .reversed()
+ .thenComparing(IMNode::getFullPath))
+ .collect(toList());
+ } finally {
+ StorageEngine.getInstance().mergeUnLock(list);
+ }
+ } catch (StorageEngineException e) {
+ throw new MetadataException(e);
+ }
+ } else {
+ // otherwise, we just sort them by the alphabetical order
+ allMatchedNodes =
+ allMatchedNodes.stream()
+ .sorted(Comparator.comparing(IMNode::getFullPath))
+ .collect(toList());
+ }
+
+ return allMatchedNodes;
+ }
+
+ /** remove the node from the tag inverted index */
+ public void removeFromTagInvertedIndex(MeasurementMNode node) throws IOException {
+ if (node.getOffset() < 0) {
+ return;
+ }
+ Map<String, String> tagMap =
+ tagLogFile.readTag(config.getTagAttributeTotalSize(), node.getOffset());
+ if (tagMap != null) {
+ for (Map.Entry<String, String> entry : tagMap.entrySet()) {
+ if (tagIndex.containsKey(entry.getKey())
+ && tagIndex.get(entry.getKey()).containsKey(entry.getValue())) {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG, "Delete" + TAG_FORMAT, node.getFullPath()),
+ entry.getKey(),
+ entry.getValue(),
+ node.getOffset()));
+ }
+ tagIndex.get(entry.getKey()).get(entry.getValue()).remove(node);
+ if (tagIndex.get(entry.getKey()).get(entry.getValue()).isEmpty()) {
+ tagIndex.get(entry.getKey()).remove(entry.getValue());
+ if (tagIndex.get(entry.getKey()).isEmpty()) {
+ tagIndex.remove(entry.getKey());
+ }
+ }
+ } else {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG_1, "Delete" + PREVIOUS_CONDITION, node.getFullPath()),
+ entry.getKey(),
+ entry.getValue(),
+ node.getOffset(),
+ tagIndex.containsKey(entry.getKey())));
+ }
+ }
+ }
+ }
+ }
+
+ /**
+ * upsert tags and attributes key-value for the timeseries if the key has existed, just use the
+ * new value to update it.
+ */
+ public void updateTagsAndAttributes(
+ Map<String, String> tagsMap, Map<String, String> attributesMap, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+
+ Pair<Map<String, String>, Map<String, String>> pair =
+ tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
+
+ if (tagsMap != null) {
+ for (Map.Entry<String, String> entry : tagsMap.entrySet()) {
+ String key = entry.getKey();
+ String value = entry.getValue();
+ String beforeValue = pair.left.get(key);
+ pair.left.put(key, value);
+ // if the key has existed and the value is not equal to the new one
+ // we should remove before key-value from inverted index map
+ if (beforeValue != null && !beforeValue.equals(value)) {
+
+ if (tagIndex.containsKey(key) && tagIndex.get(key).containsKey(beforeValue)) {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG, "Upsert" + TAG_FORMAT, leafMNode.getFullPath()),
+ key,
+ beforeValue,
+ leafMNode.getOffset()));
+ }
+
+ removeIndex(key, beforeValue, leafMNode);
+ } else {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(
+ DEBUG_MSG_1, "Upsert" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
+ key,
+ beforeValue,
+ leafMNode.getOffset(),
+ tagIndex.containsKey(key)));
+ }
+ }
+ }
+
+ // if the key doesn't exist or the value is not equal to the new one
+ // we should add a new key-value to inverted index map
+ if (beforeValue == null || !beforeValue.equals(value)) {
+ addIndex(key, value, leafMNode);
+ }
+ }
+ }
+
+ if (attributesMap != null) {
+ pair.right.putAll(attributesMap);
+ }
+
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+ }
+
+ /**
+ * add new attributes key-value for the timeseries
+ *
+ * @param attributesMap newly added attributes map
+ */
+ public void addAttributes(
+ Map<String, String> attributesMap, PartialPath fullPath, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+
+ Pair<Map<String, String>, Map<String, String>> pair =
+ tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
+
+ for (Map.Entry<String, String> entry : attributesMap.entrySet()) {
+ String key = entry.getKey();
+ String value = entry.getValue();
+ if (pair.right.containsKey(key)) {
+ throw new MetadataException(
+ String.format("TimeSeries [%s] already has the attribute [%s].", fullPath, key));
+ }
+ pair.right.put(key, value);
+ }
+
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+ }
+
+ /**
+ * add new tags key-value for the timeseries
+ *
+ * @param tagsMap newly added tags map
+ * @param fullPath timeseries
+ */
+ public void addTags(Map<String, String> tagsMap, PartialPath fullPath, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+
+ Pair<Map<String, String>, Map<String, String>> pair =
+ tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
+
+ for (Map.Entry<String, String> entry : tagsMap.entrySet()) {
+ String key = entry.getKey();
+ String value = entry.getValue();
+ if (pair.left.containsKey(key)) {
+ throw new MetadataException(
+ String.format("TimeSeries [%s] already has the tag [%s].", fullPath, key));
+ }
+ pair.left.put(key, value);
+ }
+
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+
+ // update tag inverted map
+ addIndex(tagsMap, leafMNode);
+ }
+
+ /**
+ * drop tags or attributes of the timeseries
+ *
+ * @param keySet tags key or attributes key
+ */
+ public void dropTagsOrAttributes(
+ Set<String> keySet, PartialPath fullPath, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+ Pair<Map<String, String>, Map<String, String>> pair =
+ tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
+
+ Map<String, String> deleteTag = new HashMap<>();
+ for (String key : keySet) {
+ // check tag map
+ // check attribute map
+ String removeVal = pair.left.remove(key);
+ if (removeVal != null) {
+ deleteTag.put(key, removeVal);
+ } else {
+ removeVal = pair.right.remove(key);
+ if (removeVal == null) {
+ logger.warn("TimeSeries [{}] does not have tag/attribute [{}]", fullPath, key);
+ }
+ }
+ }
+
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+
+ Map<String, Set<MeasurementMNode>> tagVal2LeafMNodeSet;
+ Set<MeasurementMNode> MMNodes;
+ for (Map.Entry<String, String> entry : deleteTag.entrySet()) {
+ String key = entry.getKey();
+ String value = entry.getValue();
+ // change the tag inverted index map
+ tagVal2LeafMNodeSet = tagIndex.get(key);
+ if (tagVal2LeafMNodeSet != null) {
+ MMNodes = tagVal2LeafMNodeSet.get(value);
+ if (MMNodes != null) {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG, "Drop" + TAG_FORMAT, leafMNode.getFullPath()),
+ entry.getKey(),
+ entry.getValue(),
+ leafMNode.getOffset()));
+ }
+
+ MMNodes.remove(leafMNode);
+ if (MMNodes.isEmpty()) {
+ tagVal2LeafMNodeSet.remove(value);
+ if (tagVal2LeafMNodeSet.isEmpty()) {
+ tagIndex.remove(key);
+ }
+ }
+ }
+ } else {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG_1, "Drop" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
+ key,
+ value,
+ leafMNode.getOffset(),
+ tagIndex.containsKey(key)));
+ }
+ }
+ }
+ }
+
+ /**
+ * set/change the values of tags or attributes
+ *
+ * @param alterMap the new tags or attributes key-value
+ */
+ public void setTagsOrAttributesValue(
+ Map<String, String> alterMap, PartialPath fullPath, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+ // tags, attributes
+ Pair<Map<String, String>, Map<String, String>> pair =
+ tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
+ Map<String, String> oldTagValue = new HashMap<>();
+ Map<String, String> newTagValue = new HashMap<>();
+
+ for (Map.Entry<String, String> entry : alterMap.entrySet()) {
+ String key = entry.getKey();
+ String value = entry.getValue();
+ // check tag map
+ if (pair.left.containsKey(key)) {
+ oldTagValue.put(key, pair.left.get(key));
+ newTagValue.put(key, value);
+ pair.left.put(key, value);
+ } else if (pair.right.containsKey(key)) {
+ // check attribute map
+ pair.right.put(key, value);
+ } else {
+ throw new MetadataException(
+ String.format("TimeSeries [%s] does not have tag/attribute [%s].", fullPath, key),
+ true);
+ }
+ }
+
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+
+ for (Map.Entry<String, String> entry : oldTagValue.entrySet()) {
+ String key = entry.getKey();
+ String beforeValue = entry.getValue();
+ String currentValue = newTagValue.get(key);
+ // change the tag inverted index map
+ if (tagIndex.containsKey(key) && tagIndex.get(key).containsKey(beforeValue)) {
+
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG, "Set" + TAG_FORMAT, leafMNode.getFullPath()),
+ entry.getKey(),
+ beforeValue,
+ leafMNode.getOffset()));
+ }
+
+ tagIndex.get(key).get(beforeValue).remove(leafMNode);
+ } else {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG_1, "Set" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
+ key,
+ beforeValue,
+ leafMNode.getOffset(),
+ tagIndex.containsKey(key)));
+ }
+ }
+ addIndex(key, currentValue, leafMNode);
+ }
+ }
+
+ /**
+ * rename the tag or attribute's key of the timeseries
+ *
+ * @param oldKey old key of tag or attribute
+ * @param newKey new key of tag or attribute
+ */
+ public void renameTagOrAttributeKey(
+ String oldKey, String newKey, PartialPath fullPath, MeasurementMNode leafMNode)
+ throws MetadataException, IOException {
+ // tags, attributes
+ Pair<Map<String, String>, Map<String, String>> pair =
+ tagLogFile.read(config.getTagAttributeTotalSize(), leafMNode.getOffset());
+
+ // current name has existed
+ if (pair.left.containsKey(newKey) || pair.right.containsKey(newKey)) {
+ throw new MetadataException(
+ String.format(
+ "TimeSeries [%s] already has a tag/attribute named [%s].", fullPath, newKey),
+ true);
+ }
+
+ // check tag map
+ if (pair.left.containsKey(oldKey)) {
+ String value = pair.left.remove(oldKey);
+ pair.left.put(newKey, value);
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+ // change the tag inverted index map
+ if (tagIndex.containsKey(oldKey) && tagIndex.get(oldKey).containsKey(value)) {
+
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(DEBUG_MSG, "Rename" + TAG_FORMAT, leafMNode.getFullPath()),
+ oldKey,
+ value,
+ leafMNode.getOffset()));
+ }
+
+ tagIndex.get(oldKey).get(value).remove(leafMNode);
+
+ } else {
+ if (logger.isDebugEnabled()) {
+ logger.debug(
+ String.format(
+ String.format(
+ DEBUG_MSG_1, "Rename" + PREVIOUS_CONDITION, leafMNode.getFullPath()),
+ oldKey,
+ value,
+ leafMNode.getOffset(),
+ tagIndex.containsKey(oldKey)));
+ }
+ }
+ addIndex(newKey, value, leafMNode);
+ } else if (pair.right.containsKey(oldKey)) {
+ // check attribute map
+ pair.right.put(newKey, pair.right.remove(oldKey));
+ // persist the change to disk
+ tagLogFile.write(pair.left, pair.right, leafMNode.getOffset());
+ } else {
+ throw new MetadataException(
+ String.format("TimeSeries [%s] does not have tag/attribute [%s].", fullPath, oldKey),
+ true);
+ }
+ }
+
+ public long writeTagFile(Map<String, String> tags, Map<String, String> attributes)
+ throws MetadataException, IOException {
+ return tagLogFile.write(tags, attributes);
+ }
+
+ public Pair<Map<String, String>, Map<String, String>> readTagFile(long tagFileOffset)
+ throws IOException {
+ return tagLogFile.read(config.getTagAttributeTotalSize(), tagFileOffset);
+ }
+
+ public void clear() throws IOException {
+ this.tagIndex.clear();
+ if (tagLogFile != null) {
+ tagLogFile.close();
+ tagLogFile = null;
+ }
+ }
+}
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/template/Template.java b/server/src/main/java/org/apache/iotdb/db/metadata/template/Template.java
index 5a59c52..1612483 100644
--- a/server/src/main/java/org/apache/iotdb/db/metadata/template/Template.java
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/template/Template.java
@@ -120,6 +120,10 @@ public class Template {
|| schemaMap.containsKey(path.getDevicePath().getMeasurement()));
}
+ public boolean hasSchema(String measurementId) {
+ return schemaMap.containsKey(measurementId);
+ }
+
public List<MeasurementMNode> getMeasurementMNode() {
Set<IMeasurementSchema> deduplicateSchema = new HashSet<>();
List<MeasurementMNode> res = new ArrayList<>();
diff --git a/server/src/main/java/org/apache/iotdb/db/metadata/template/TemplateManager.java b/server/src/main/java/org/apache/iotdb/db/metadata/template/TemplateManager.java
new file mode 100644
index 0000000..d119c39
--- /dev/null
+++ b/server/src/main/java/org/apache/iotdb/db/metadata/template/TemplateManager.java
@@ -0,0 +1,141 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.iotdb.db.metadata.template;
+
+import org.apache.iotdb.db.exception.metadata.DuplicatedTemplateException;
+import org.apache.iotdb.db.exception.metadata.MetadataException;
+import org.apache.iotdb.db.exception.metadata.PathAlreadyExistException;
+import org.apache.iotdb.db.exception.metadata.UndefinedTemplateException;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
+import org.apache.iotdb.db.qp.physical.crud.CreateTemplatePlan;
+import org.apache.iotdb.db.utils.TestOnly;
+import org.apache.iotdb.tsfile.utils.Pair;
+import org.apache.iotdb.tsfile.write.schema.IMeasurementSchema;
+
+import java.util.HashMap;
+import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
+
+public class TemplateManager {
+
+ // template name -> template
+ private Map<String, Template> templateMap = new ConcurrentHashMap<>();
+
+ private static class TemplateManagerHolder {
+
+ private TemplateManagerHolder() {
+ // allowed to do nothing
+ }
+
+ private static final TemplateManager INSTANCE = new TemplateManager();
+ }
+
+ public static TemplateManager getInstance() {
+ return TemplateManagerHolder.INSTANCE;
+ }
+
+ @TestOnly
+ public static TemplateManager getNewInstanceForTest() {
+ return new TemplateManager();
+ }
+
+ private TemplateManager() {}
+
+ public void createDeviceTemplate(CreateTemplatePlan plan) throws MetadataException {
+ Template template = new Template(plan);
+ if (templateMap.putIfAbsent(plan.getName(), template) != null) {
+ // already have template
+ throw new MetadataException("Duplicated template name: " + plan.getName());
+ }
+ }
+
+ public Template getTemplate(String templateName) throws UndefinedTemplateException {
+ Template template = templateMap.get(templateName);
+ if (template == null) {
+ throw new UndefinedTemplateException(templateName);
+ }
+ return template;
+ }
+
+ public void setDeviceTemplate(Template template, Pair<IMNode, Template> node)
+ throws MetadataException {
+
+ if (node.left.getDeviceTemplate() != null) {
+ if (node.left.getDeviceTemplate().equals(template)) {
+ throw new DuplicatedTemplateException(template.getName());
+ } else {
+ throw new MetadataException("Specified node already has template");
+ }
+ }
+
+ if (!isTemplateCompatible(node.right, template)) {
+ throw new MetadataException("Incompatible template");
+ }
+
+ checkIsTemplateAndMNodeCompatible(template, node.left);
+
+ node.left.setDeviceTemplate(template);
+ }
+
+ public boolean isTemplateCompatible(Template upper, Template current) {
+ if (upper == null) {
+ return true;
+ }
+
+ Map<String, IMeasurementSchema> upperMap = new HashMap<>(upper.getSchemaMap());
+ Map<String, IMeasurementSchema> currentMap = new HashMap<>(current.getSchemaMap());
+
+ // for identical vector schema, we should just compare once
+ Map<IMeasurementSchema, IMeasurementSchema> sameSchema = new HashMap<>();
+
+ for (String name : currentMap.keySet()) {
+ IMeasurementSchema upperSchema = upperMap.remove(name);
+ if (upperSchema != null) {
+ IMeasurementSchema currentSchema = currentMap.get(name);
+ // use "==" to compare actual address space
+ if (upperSchema == sameSchema.get(currentSchema)) {
+ continue;
+ }
+
+ if (!upperSchema.equals(currentSchema)) {
+ return false;
+ }
+
+ sameSchema.put(currentSchema, upperSchema);
+ }
+ }
+
+ // current template must contains all measurements of upper template
+ return upperMap.isEmpty();
+ }
+
+ public void checkIsTemplateAndMNodeCompatible(Template template, IMNode IMNode)
+ throws PathAlreadyExistException {
+ for (String schemaName : template.getSchemaMap().keySet()) {
+ if (IMNode.hasChild(schemaName)) {
+ throw new PathAlreadyExistException(
+ IMNode.getPartialPath().concatNode(schemaName).getFullPath());
+ }
+ }
+ }
+
+ public void clear() {
+ templateMap.clear();
+ }
+}
diff --git a/server/src/main/java/org/apache/iotdb/db/qp/executor/PlanExecutor.java b/server/src/main/java/org/apache/iotdb/db/qp/executor/PlanExecutor.java
index 2690392..b778a49 100644
--- a/server/src/main/java/org/apache/iotdb/db/qp/executor/PlanExecutor.java
+++ b/server/src/main/java/org/apache/iotdb/db/qp/executor/PlanExecutor.java
@@ -50,7 +50,7 @@ import org.apache.iotdb.db.exception.metadata.PathNotExistException;
import org.apache.iotdb.db.exception.metadata.StorageGroupNotSetException;
import org.apache.iotdb.db.exception.query.QueryProcessException;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.metadata.mnode.StorageGroupMNode;
import org.apache.iotdb.db.monitor.StatMonitor;
@@ -1184,7 +1184,7 @@ public class PlanExecutor implements IPlanExecutor {
Set<PartialPath> registeredSeries = new HashSet<>();
for (ChunkGroupMetadata chunkGroupMetadata : chunkGroupMetadataList) {
String device = chunkGroupMetadata.getDevice();
- MNode node =
+ IMNode node =
IoTDB.metaManager.getDeviceNodeWithAutoCreate(
new PartialPath(device), true, true, sgLevel)
.left;
@@ -1281,7 +1281,7 @@ public class PlanExecutor implements IPlanExecutor {
}
}
- protected MNode getSeriesSchemas(InsertPlan insertPlan) throws MetadataException {
+ protected IMNode getSeriesSchemas(InsertPlan insertPlan) throws MetadataException {
try {
return IoTDB.metaManager.getSeriesSchemasAndReadLockDevice(insertPlan);
} catch (IOException e) {
diff --git a/server/src/main/java/org/apache/iotdb/db/query/dataset/AlignByDeviceDataSet.java b/server/src/main/java/org/apache/iotdb/db/query/dataset/AlignByDeviceDataSet.java
index a34dd0b..923f459 100644
--- a/server/src/main/java/org/apache/iotdb/db/query/dataset/AlignByDeviceDataSet.java
+++ b/server/src/main/java/org/apache/iotdb/db/query/dataset/AlignByDeviceDataSet.java
@@ -23,7 +23,7 @@ import org.apache.iotdb.db.exception.StorageEngineException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.exception.query.QueryProcessException;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.template.Template;
import org.apache.iotdb.db.qp.physical.crud.AggregationPlan;
import org.apache.iotdb.db.qp.physical.crud.AlignByDevicePlan;
@@ -229,9 +229,9 @@ public class AlignByDeviceDataSet extends QueryDataSet {
protected Set<String> getDeviceMeasurements(PartialPath device) throws IOException {
try {
- MNode deviceNode = IoTDB.metaManager.getNodeByPath(device);
+ IMNode deviceNode = IoTDB.metaManager.getNodeByPath(device);
Set<String> res = new HashSet<>(deviceNode.getChildren().keySet());
- for (MNode mnode : deviceNode.getChildren().values()) {
+ for (IMNode mnode : deviceNode.getChildren().values()) {
res.addAll(mnode.getChildren().keySet());
}
diff --git a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAddSubDeviceIT.java b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAddSubDeviceIT.java
index cb03f85..67b0da8 100644
--- a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAddSubDeviceIT.java
+++ b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAddSubDeviceIT.java
@@ -21,10 +21,7 @@ package org.apache.iotdb.db.integration;
import org.apache.iotdb.db.utils.EnvironmentUtils;
import org.apache.iotdb.jdbc.Config;
-import org.junit.AfterClass;
-import org.junit.Assert;
-import org.junit.BeforeClass;
-import org.junit.Test;
+import org.junit.*;
import java.sql.Connection;
import java.sql.DriverManager;
@@ -35,7 +32,11 @@ import java.sql.Types;
import static org.junit.Assert.fail;
-/** Test if measurement is also a sub device. */
+/**
+ * Test if measurement is also a sub device. This test will be ignored since nested measurement has
+ * been removed from v0.13
+ */
+@Ignore // nested measurement has been forbidden
public class IoTDBAddSubDeviceIT {
private static String[] sqls =
diff --git a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAutoCreateSchemaIT.java b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAutoCreateSchemaIT.java
index abac508..2b38036 100644
--- a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAutoCreateSchemaIT.java
+++ b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBAutoCreateSchemaIT.java
@@ -24,10 +24,7 @@ import org.apache.iotdb.db.utils.EnvironmentUtils;
import org.apache.iotdb.jdbc.Config;
import org.apache.iotdb.jdbc.IoTDBSQLException;
-import org.junit.After;
-import org.junit.Assert;
-import org.junit.Before;
-import org.junit.Test;
+import org.junit.*;
import java.sql.Connection;
import java.sql.DatabaseMetaData;
@@ -159,6 +156,7 @@ public class IoTDBAutoCreateSchemaIT {
* insert data when the time series that is a prefix path of an existing time series hasn't been
* created
*/
+ @Ignore // nested measurement has been forbidden
@Test
public void testInsertAutoCreate1() throws Exception {
String[] timeSeriesArray = {"root.sg1.a.a", "root.sg1.a", "root.sg1.a.a.a"};
diff --git a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBCreateTimeseriesIT.java b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBCreateTimeseriesIT.java
index 5174419..12662f4 100644
--- a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBCreateTimeseriesIT.java
+++ b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBCreateTimeseriesIT.java
@@ -23,10 +23,7 @@ import org.apache.iotdb.db.utils.EnvironmentUtils;
import org.apache.iotdb.jdbc.Config;
import org.apache.iotdb.jdbc.IoTDBSQLException;
-import org.junit.After;
-import org.junit.Assert;
-import org.junit.Before;
-import org.junit.Test;
+import org.junit.*;
import java.sql.Connection;
import java.sql.DriverManager;
@@ -65,25 +62,30 @@ public class IoTDBCreateTimeseriesIT {
}
/** Test creating a time series that is a prefix path of an existing time series */
+ @Ignore // nested measurement has been forbidden
@Test
public void testCreateTimeseries1() throws Exception {
String[] timeSeriesArray = {"root.sg1.aa.bb", "root.sg1.aa.bb.cc", "root.sg1.aa"};
- for (String timeSeries : timeSeriesArray) {
- statement.execute(
- String.format(
- "create timeseries %s with datatype=INT64, encoding=PLAIN, compression=SNAPPY",
- timeSeries));
- }
+ try {
+ for (String timeSeries : timeSeriesArray) {
+ statement.execute(
+ String.format(
+ "create timeseries %s with datatype=INT64, encoding=PLAIN, compression=SNAPPY",
+ timeSeries));
+ }
- // ensure that current timeseries in cache is right.
- createTimeSeries1Tool(timeSeriesArray);
+ // ensure that current timeseries in cache is right.
+ createTimeSeries1Tool(timeSeriesArray);
- EnvironmentUtils.stopDaemon();
- setUp();
+ EnvironmentUtils.stopDaemon();
+ setUp();
- // ensure timeseries in cache is right after recovering.
- createTimeSeries1Tool(timeSeriesArray);
+ // ensure timeseries in cache is right after recovering.
+ createTimeSeries1Tool(timeSeriesArray);
+ } catch (IoTDBSQLException e) {
+ Assert.assertEquals("300: Path [root.sg1.aa.bb] already exist", e.getMessage());
+ }
}
private void createTimeSeries1Tool(String[] timeSeriesArray) throws SQLException {
diff --git a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBLastIT.java b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBLastIT.java
index c1327e3..3dabe36 100644
--- a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBLastIT.java
+++ b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBLastIT.java
@@ -20,7 +20,7 @@ package org.apache.iotdb.db.integration;
import org.apache.iotdb.db.exception.metadata.MetadataException;
import org.apache.iotdb.db.metadata.PartialPath;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.service.IoTDB;
import org.apache.iotdb.db.utils.EnvironmentUtils;
@@ -283,7 +283,7 @@ public class IoTDBLastIT {
DriverManager.getConnection("jdbc:iotdb://127.0.0.1:6667/", "root", "root");
Statement statement = connection.createStatement()) {
- MNode node =
+ IMNode node =
IoTDB.metaManager.getNodeByPath(new PartialPath("root.ln.wf01.wt02.temperature"));
((MeasurementMNode) node).resetCache();
boolean hasResultSet =
@@ -367,7 +367,7 @@ public class IoTDBLastIT {
DriverManager.getConnection("jdbc:iotdb://127.0.0.1:6667/", "root", "root");
Statement statement = connection.createStatement()) {
- MNode node =
+ IMNode node =
IoTDB.metaManager.getNodeByPath(new PartialPath("root.ln.wf01.wt03.temperature"));
((MeasurementMNode) node).resetCache();
@@ -416,7 +416,7 @@ public class IoTDBLastIT {
statement.execute("INSERT INTO root.ln.wf01.wt04(timestamp,temperature) values(150,31.2)");
statement.execute("flush");
- MNode node =
+ IMNode node =
IoTDB.metaManager.getNodeByPath(new PartialPath("root.ln.wf01.wt04.temperature"));
((MeasurementMNode) node).resetCache();
diff --git a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBMetadataFetchIT.java b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBMetadataFetchIT.java
index de1f43b..6f96df7 100644
--- a/server/src/test/java/org/apache/iotdb/db/integration/IoTDBMetadataFetchIT.java
+++ b/server/src/test/java/org/apache/iotdb/db/integration/IoTDBMetadataFetchIT.java
@@ -24,10 +24,7 @@ import org.apache.iotdb.jdbc.Config;
import com.google.gson.Gson;
import com.google.gson.JsonObject;
-import org.junit.After;
-import org.junit.Assert;
-import org.junit.Before;
-import org.junit.Test;
+import org.junit.*;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@@ -68,7 +65,6 @@ public class IoTDBMetadataFetchIT {
"SET STORAGE GROUP TO root.ln1.wf01.wt01",
"SET STORAGE GROUP TO root.ln2.wf01.wt01",
"CREATE TIMESERIES root.ln.wf01.wt01.status WITH DATATYPE = BOOLEAN, ENCODING = PLAIN",
- "CREATE TIMESERIES root.ln.wf01.wt01.status.s1 WITH DATATYPE = BOOLEAN, ENCODING = PLAIN",
"CREATE TIMESERIES root.ln.wf01.wt01.temperature WITH DATATYPE = FLOAT, ENCODING = RLE, "
+ "compressor = SNAPPY, MAX_POINT_NUMBER = 3"
};
@@ -115,22 +111,18 @@ public class IoTDBMetadataFetchIT {
new Set[] {
new HashSet<>(
Arrays.asList(
- "root.ln.wf01.wt01.status,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
- "root.ln.wf01.wt01.status.s1,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,")),
+ "root.ln.wf01.wt01.status,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,")),
new HashSet<>(
Arrays.asList(
"root.ln.wf01.wt01.status,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
- "root.ln.wf01.wt01.status.s1,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
"root.ln.wf01.wt01.temperature,null,root.ln.wf01.wt01,FLOAT,RLE,SNAPPY,null,null,")),
new HashSet<>(
Arrays.asList(
"root.ln.wf01.wt01.status,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
- "root.ln.wf01.wt01.status.s1,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
"root.ln.wf01.wt01.temperature,null,root.ln.wf01.wt01,FLOAT,RLE,SNAPPY,null,null,")),
new HashSet<>(
Arrays.asList(
"root.ln.wf01.wt01.status,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
- "root.ln.wf01.wt01.status.s1,null,root.ln.wf01.wt01,BOOLEAN,PLAIN,SNAPPY,null,null,",
"root.ln.wf01.wt01.temperature,null,root.ln.wf01.wt01,FLOAT,RLE,SNAPPY,null,null,")),
new HashSet<>(Collections.singletonList(""))
};
@@ -410,7 +402,7 @@ public class IoTDBMetadataFetchIT {
Config.IOTDB_URL_PREFIX + "127.0.0.1:6667/", "root", "root");
Statement statement = connection.createStatement()) {
String[] sqls = new String[] {"COUNT TIMESERIES root.ln", "COUNT TIMESERIES"};
- String[] standards = new String[] {"3,\n", "3,\n"};
+ String[] standards = new String[] {"2,\n", "2,\n"};
for (int n = 0; n < sqls.length; n++) {
String sql = sqls[n];
String standard = standards[n];
@@ -448,7 +440,7 @@ public class IoTDBMetadataFetchIT {
new String[] {
"COUNT DEVICES root.ln", "COUNT DEVICES", "COUNT DEVICES root.ln.wf01.wt01.temperature"
};
- String[] standards = new String[] {"2,\n", "2,\n", "0,\n"};
+ String[] standards = new String[] {"1,\n", "1,\n", "0,\n"};
for (int n = 0; n < sqls.length; n++) {
String sql = sqls[n];
String standard = standards[n];
@@ -524,7 +516,7 @@ public class IoTDBMetadataFetchIT {
Statement statement = connection.createStatement()) {
String[] sqls = new String[] {"COUNT TIMESERIES root group by level=1"};
Set<String>[] standards =
- new Set[] {new HashSet<>(Arrays.asList("root.ln,3,", "root.ln1,0,", "root.ln2,0,"))};
+ new Set[] {new HashSet<>(Arrays.asList("root.ln,2,", "root.ln1,0,", "root.ln2,0,"))};
for (int n = 0; n < sqls.length; n++) {
String sql = sqls[n];
Set<String> standard = standards[n];
@@ -607,12 +599,10 @@ public class IoTDBMetadataFetchIT {
+ "\t\t\t\t\t\t\"Encoding\":\"RLE\"\n"
+ "\t\t\t\t\t},\n"
+ "\t\t\t\t\t\"status\":{\n"
- + "\t\t\t\t\t\t\"s1\":{\n"
+ "\t\t\t\t\t\t\t\"StorageGroup\":\"root.ln.wf01.wt01\",\n"
+ "\t\t\t\t\t\t\t\"DataType\":\"BOOLEAN\",\n"
+ "\t\t\t\t\t\t\t\"Compressor\":\"SNAPPY\",\n"
+ "\t\t\t\t\t\t\t\"Encoding\":\"PLAIN\"\n"
- + "\t\t\t\t\t\t}\n"
+ "\t\t\t\t\t}\n"
+ "\t\t\t\t}\n"
+ "\t\t\t}\n"
diff --git a/server/src/test/java/org/apache/iotdb/db/metadata/MManagerAdvancedTest.java b/server/src/test/java/org/apache/iotdb/db/metadata/MManagerAdvancedTest.java
index e8d1068..0f6bb41 100644
--- a/server/src/test/java/org/apache/iotdb/db/metadata/MManagerAdvancedTest.java
+++ b/server/src/test/java/org/apache/iotdb/db/metadata/MManagerAdvancedTest.java
@@ -19,7 +19,7 @@
package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.exception.metadata.MetadataException;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.service.IoTDB;
import org.apache.iotdb.db.utils.EnvironmentUtils;
@@ -195,7 +195,7 @@ public class MManagerAdvancedTest {
TSFileDescriptor.getInstance().getConfig().getCompressor(),
Collections.emptyMap());
- MNode node = mmanager.getNodeByPath(new PartialPath("root.vehicle.d0"));
+ IMNode node = mmanager.getNodeByPath(new PartialPath("root.vehicle.d0"));
Assert.assertEquals(
TSDataType.INT32, ((MeasurementMNode) node.getChild("s0")).getSchema().getType());
@@ -218,7 +218,7 @@ public class MManagerAdvancedTest {
TimeValuePair tv1 = new TimeValuePair(1000, TsPrimitiveType.getByType(TSDataType.DOUBLE, 1.0));
TimeValuePair tv2 = new TimeValuePair(2000, TsPrimitiveType.getByType(TSDataType.DOUBLE, 3.0));
TimeValuePair tv3 = new TimeValuePair(1500, TsPrimitiveType.getByType(TSDataType.DOUBLE, 2.5));
- MNode node = mmanager.getNodeByPath(new PartialPath("root.vehicle.d2.s0"));
+ IMNode node = mmanager.getNodeByPath(new PartialPath("root.vehicle.d2.s0"));
((MeasurementMNode) node).updateCachedLast(tv1, true, Long.MIN_VALUE);
((MeasurementMNode) node).updateCachedLast(tv2, true, Long.MIN_VALUE);
Assert.assertEquals(
diff --git a/server/src/test/java/org/apache/iotdb/db/metadata/MManagerBasicTest.java b/server/src/test/java/org/apache/iotdb/db/metadata/MManagerBasicTest.java
index 9203857..00632f5 100644
--- a/server/src/test/java/org/apache/iotdb/db/metadata/MManagerBasicTest.java
+++ b/server/src/test/java/org/apache/iotdb/db/metadata/MManagerBasicTest.java
@@ -21,8 +21,9 @@ package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.conf.IoTDBDescriptor;
import org.apache.iotdb.db.exception.metadata.IllegalPathException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
+import org.apache.iotdb.db.exception.metadata.PathAlreadyExistException;
import org.apache.iotdb.db.exception.metadata.StorageGroupNotSetException;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.metadata.template.Template;
import org.apache.iotdb.db.qp.physical.crud.CreateTemplatePlan;
@@ -347,7 +348,7 @@ public class MManagerBasicTest {
try {
manager.setStorageGroup(new PartialPath("root.laptop"));
manager.createTimeseries(
- new PartialPath("root.laptop.d1"),
+ new PartialPath("root.laptop.d0"),
TSDataType.INT32,
TSEncoding.PLAIN,
CompressionType.GZIP,
@@ -359,13 +360,13 @@ public class MManagerBasicTest {
CompressionType.GZIP,
null);
manager.createTimeseries(
- new PartialPath("root.laptop.d1.s1.t1"),
+ new PartialPath("root.laptop.d1.s2.t1"),
TSDataType.INT32,
TSEncoding.PLAIN,
CompressionType.GZIP,
null);
manager.createTimeseries(
- new PartialPath("root.laptop.d1.s2"),
+ new PartialPath("root.laptop.d1.s3"),
TSDataType.INT32,
TSEncoding.PLAIN,
CompressionType.GZIP,
@@ -388,8 +389,8 @@ public class MManagerBasicTest {
assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.*")), 6);
assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.*.*")), 5);
assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.*.*.t1")), 1);
- assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.*.s1")), 3);
- assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.d1")), 4);
+ assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.*.s1")), 2);
+ assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.d1")), 3);
assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.d1.*")), 3);
assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.d2.s1")), 1);
assertEquals(manager.getAllTimeseriesCount(new PartialPath("root.laptop.d2")), 2);
@@ -485,7 +486,7 @@ public class MManagerBasicTest {
.collect(Collectors.toSet()));
MManager recoverManager = new MManager();
- recoverManager.init();
+ recoverManager.initForMultiMManagerTest();
assertTrue(recoverManager.isStorageGroup(new PartialPath("root.laptop.d1")));
assertFalse(recoverManager.isStorageGroup(new PartialPath("root.laptop.d2")));
@@ -860,7 +861,7 @@ public class MManagerBasicTest {
manager.setDeviceTemplate(setDeviceTemplatePlan);
- MNode node = manager.getDeviceNode(new PartialPath("root.sg1.d1"));
+ IMNode node = manager.getDeviceNode(new PartialPath("root.sg1.d1"));
node.setUseTemplate(true);
MeasurementSchema s11 =
@@ -1089,6 +1090,58 @@ public class MManagerBasicTest {
}
@Test
+ public void testTemplateAndNodePathCompatibility() throws MetadataException {
+ MManager manager = IoTDB.metaManager;
+ CreateTemplatePlan plan = getCreateTemplatePlan();
+ manager.createDeviceTemplate(plan);
+
+ // set device template
+ SetDeviceTemplatePlan setDeviceTemplatePlan =
+ new SetDeviceTemplatePlan("template1", "root.sg1.d1");
+
+ CreateTimeSeriesPlan createTimeSeriesPlan =
+ new CreateTimeSeriesPlan(
+ new PartialPath("root.sg1.d1.s11"),
+ TSDataType.INT32,
+ TSEncoding.PLAIN,
+ CompressionType.GZIP,
+ null,
+ null,
+ null,
+ null);
+
+ manager.createTimeseries(createTimeSeriesPlan);
+
+ try {
+ manager.setDeviceTemplate(setDeviceTemplatePlan);
+ fail();
+ } catch (PathAlreadyExistException e) {
+ assertEquals("Path [root.sg1.d1.s11] already exist", e.getMessage());
+ }
+
+ manager.deleteTimeseries(new PartialPath("root.sg1.d1.s11"));
+
+ CreateTimeSeriesPlan createTimeSeriesPlan2 =
+ new CreateTimeSeriesPlan(
+ new PartialPath("root.sg1.d1.vector.s1"),
+ TSDataType.INT32,
+ TSEncoding.PLAIN,
+ CompressionType.GZIP,
+ null,
+ null,
+ null,
+ null);
+ manager.createTimeseries(createTimeSeriesPlan2);
+
+ try {
+ manager.setDeviceTemplate(setDeviceTemplatePlan);
+ fail();
+ } catch (PathAlreadyExistException e) {
+ assertEquals("Path [root.sg1.d1.vector] already exist", e.getMessage());
+ }
+ }
+
+ @Test
public void testShowTimeseries() {
MManager manager = IoTDB.metaManager;
try {
@@ -1396,7 +1449,7 @@ public class MManagerBasicTest {
try {
manager.setStorageGroup(new PartialPath("root.laptop"));
manager.createTimeseries(
- new PartialPath("root.laptop.d1"),
+ new PartialPath("root.laptop.d0"),
TSDataType.INT32,
TSEncoding.PLAIN,
CompressionType.GZIP,
@@ -1408,13 +1461,13 @@ public class MManagerBasicTest {
CompressionType.GZIP,
null);
manager.createTimeseries(
- new PartialPath("root.laptop.d1.s1.t1"),
+ new PartialPath("root.laptop.d1.s2.t1"),
TSDataType.INT32,
TSEncoding.PLAIN,
CompressionType.GZIP,
null);
manager.createTimeseries(
- new PartialPath("root.laptop.d1.s2"),
+ new PartialPath("root.laptop.d1.s3"),
TSDataType.INT32,
TSEncoding.PLAIN,
CompressionType.GZIP,
@@ -1500,8 +1553,9 @@ public class MManagerBasicTest {
new MeasurementMNode[insertRowPlan.getMeasurements().length]);
// call getSeriesSchemasAndReadLockDevice
- MNode mNode = manager.getSeriesSchemasAndReadLockDevice(insertRowPlan);
- assertEquals(4, mNode.getMeasurementMNodeCount());
+ IMNode IMNode = manager.getSeriesSchemasAndReadLockDevice(insertRowPlan);
+ assertEquals(3, manager.getAllTimeseriesCount(IMNode.getPartialPath()));
+ assertEquals(1, IMNode.getMeasurementMNodeCount());
assertNull(insertRowPlan.getMeasurementMNodes()[0]);
assertNull(insertRowPlan.getMeasurementMNodes()[1]);
assertNull(insertRowPlan.getMeasurementMNodes()[2]);
@@ -1586,8 +1640,8 @@ public class MManagerBasicTest {
new MeasurementMNode[insertRowPlan.getMeasurements().length]);
// call getSeriesSchemasAndReadLockDevice
- MNode mNode = manager.getSeriesSchemasAndReadLockDevice(insertRowPlan);
- assertEquals(1, mNode.getMeasurementMNodeCount());
+ IMNode IMNode = manager.getSeriesSchemasAndReadLockDevice(insertRowPlan);
+ assertEquals(1, IMNode.getMeasurementMNodeCount());
assertNull(insertRowPlan.getMeasurementMNodes()[0]);
assertEquals(1, insertRowPlan.getFailedMeasurementNumber());
diff --git a/server/src/test/java/org/apache/iotdb/db/metadata/MManagerImproveTest.java b/server/src/test/java/org/apache/iotdb/db/metadata/MManagerImproveTest.java
index 4b52b54..effcb24 100644
--- a/server/src/test/java/org/apache/iotdb/db/metadata/MManagerImproveTest.java
+++ b/server/src/test/java/org/apache/iotdb/db/metadata/MManagerImproveTest.java
@@ -20,7 +20,7 @@ package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.exception.metadata.IllegalPathException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.service.IoTDB;
import org.apache.iotdb.db.utils.EnvironmentUtils;
@@ -138,7 +138,7 @@ public class MManagerImproveTest {
private void doCacheTest(String deviceId, List<String> measurementList) throws MetadataException {
try {
- MNode node = mManager.getDeviceNodeWithAutoCreate(new PartialPath(deviceId)).left;
+ IMNode node = mManager.getDeviceNodeWithAutoCreate(new PartialPath(deviceId)).left;
for (String s : measurementList) {
assertTrue(node.hasChild(s));
MeasurementMNode measurementNode = (MeasurementMNode) node.getChild(s);
diff --git a/server/src/test/java/org/apache/iotdb/db/metadata/MTreeTest.java b/server/src/test/java/org/apache/iotdb/db/metadata/MTreeTest.java
index a9a1b9d..2070c69 100644
--- a/server/src/test/java/org/apache/iotdb/db/metadata/MTreeTest.java
+++ b/server/src/test/java/org/apache/iotdb/db/metadata/MTreeTest.java
@@ -21,7 +21,8 @@ package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.exception.metadata.AliasAlreadyExistException;
import org.apache.iotdb.db.exception.metadata.IllegalPathException;
import org.apache.iotdb.db.exception.metadata.MetadataException;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.exception.metadata.PathAlreadyExistException;
+import org.apache.iotdb.db.metadata.mnode.IMNode;
import org.apache.iotdb.db.metadata.mnode.MeasurementMNode;
import org.apache.iotdb.db.utils.EnvironmentUtils;
import org.apache.iotdb.tsfile.common.conf.TSFileDescriptor;
@@ -710,7 +711,7 @@ public class MTreeTest {
Collections.emptyMap(),
null);
root.createTimeseries(
- new PartialPath("root.laptop.d1.s1.b"),
+ new PartialPath("root.laptop.d1.s2.b"),
TSDataType.INT32,
TSEncoding.RLE,
TSFileDescriptor.getInstance().getConfig().getCompressor(),
@@ -767,44 +768,21 @@ public class MTreeTest {
}
@Test
- public void testDeleteChildOfMeasurementMNode() throws MetadataException {
+ public void testCreateTimeseries() throws MetadataException {
MTree root = new MTree();
String sgPath = "root.sg1";
root.setStorageGroup(new PartialPath(sgPath));
- try {
- root.createTimeseries(
- new PartialPath("root.sg1.a.b"),
- TSDataType.INT32,
- TSEncoding.RLE,
- TSFileDescriptor.getInstance().getConfig().getCompressor(),
- Collections.emptyMap(),
- null);
- root.createTimeseries(
- new PartialPath("root.sg1.a.b.c"),
- TSDataType.INT32,
- TSEncoding.RLE,
- TSFileDescriptor.getInstance().getConfig().getCompressor(),
- Collections.emptyMap(),
- null);
- assertTrue(root.isPathExist(new PartialPath("root.sg1.a.b")));
- assertTrue(root.isPathExist(new PartialPath("root.sg1.a.b.c")));
-
- root.deleteTimeseriesAndReturnEmptyStorageGroup(new PartialPath("root.sg1.a.b.c"));
- assertFalse(root.isPathExist(new PartialPath("root.sg1.a.b.c")));
- assertTrue(root.isPathExist(new PartialPath("root.sg1.a.b")));
-
- } catch (MetadataException e1) {
- fail(e1.getMessage());
- }
- }
+ root.createTimeseries(
+ new PartialPath("root.sg1.a.b.c"),
+ TSDataType.INT32,
+ TSEncoding.RLE,
+ TSFileDescriptor.getInstance().getConfig().getCompressor(),
+ Collections.emptyMap(),
+ null);
- @Test
- public void testGetMeasurementMNodeCount() throws MetadataException {
- MTree root = new MTree();
- PartialPath sgPath = new PartialPath("root.sg1");
- root.setStorageGroup(sgPath);
try {
+ // mtree doesn't support nested timeseries which means MeasurementMNode is leaf of the tree.
root.createTimeseries(
new PartialPath("root.sg1.a.b"),
TSDataType.INT32,
@@ -812,60 +790,11 @@ public class MTreeTest {
TSFileDescriptor.getInstance().getConfig().getCompressor(),
Collections.emptyMap(),
null);
- MNode sgNode = root.getNodeByPath(sgPath);
- assertEquals(1, sgNode.getMeasurementMNodeCount()); // b
-
- root.createTimeseries(
- new PartialPath("root.sg1.a.b.c"),
- TSDataType.INT32,
- TSEncoding.RLE,
- TSFileDescriptor.getInstance().getConfig().getCompressor(),
- Collections.emptyMap(),
- null);
- assertEquals(2, sgNode.getMeasurementMNodeCount()); // b and c
- MNode cNode = sgNode.getChild("a").getChild("b").getChild("c");
- assertEquals(1, cNode.getMeasurementMNodeCount()); // c
-
- root.createTimeseries(
- new PartialPath("root.sg1.a.b.c.d"),
- TSDataType.INT32,
- TSEncoding.RLE,
- TSFileDescriptor.getInstance().getConfig().getCompressor(),
- Collections.emptyMap(),
- null);
- assertEquals(3, sgNode.getMeasurementMNodeCount()); // b, c and d
- assertEquals(2, cNode.getMeasurementMNodeCount()); // c and d
- MNode dNode = cNode.getChild("d");
- assertEquals(1, dNode.getMeasurementMNodeCount()); // d
-
- } catch (MetadataException e1) {
- fail(e1.getMessage());
+ } catch (PathAlreadyExistException e) {
+ assertEquals("Path [root.sg1.a.b] already exist", e.getMessage());
}
- }
-
- @Test
- public void testCreateTimeseries() throws MetadataException {
- MTree root = new MTree();
- String sgPath = "root.sg1";
- root.setStorageGroup(new PartialPath(sgPath));
-
- root.createTimeseries(
- new PartialPath("root.sg1.a.b.c"),
- TSDataType.INT32,
- TSEncoding.RLE,
- TSFileDescriptor.getInstance().getConfig().getCompressor(),
- Collections.emptyMap(),
- null);
-
- root.createTimeseries(
- new PartialPath("root.sg1.a.b"),
- TSDataType.INT32,
- TSEncoding.RLE,
- TSFileDescriptor.getInstance().getConfig().getCompressor(),
- Collections.emptyMap(),
- null);
- MNode node = root.getNodeByPath(new PartialPath("root.sg1.a.b"));
- Assert.assertTrue(node instanceof MeasurementMNode);
+ IMNode node = root.getNodeByPath(new PartialPath("root.sg1.a.b"));
+ Assert.assertFalse(node instanceof MeasurementMNode);
}
}
diff --git a/server/src/test/java/org/apache/iotdb/db/metadata/MetaUtilsTest.java b/server/src/test/java/org/apache/iotdb/db/metadata/MetaUtilsTest.java
index 49d85a2..61e72b5 100644
--- a/server/src/test/java/org/apache/iotdb/db/metadata/MetaUtilsTest.java
+++ b/server/src/test/java/org/apache/iotdb/db/metadata/MetaUtilsTest.java
@@ -19,7 +19,7 @@
package org.apache.iotdb.db.metadata;
import org.apache.iotdb.db.exception.metadata.IllegalPathException;
-import org.apache.iotdb.db.metadata.mnode.MNode;
+import org.apache.iotdb.db.metadata.mnode.InternalMNode;
import org.junit.Assert;
import org.junit.Test;
@@ -81,22 +81,22 @@ public class MetaUtilsTest {
@Test
public void testGetMultiFullPaths() {
- MNode rootNode = new MNode(null, "root");
+ InternalMNode rootNode = new InternalMNode(null, "root");
// builds the relationship of root.a and root.aa
- MNode aNode = new MNode(rootNode, "a");
+ InternalMNode aNode = new InternalMNode(rootNode, "a");
rootNode.addChild(aNode.getName(), aNode);
- MNode aaNode = new MNode(rootNode, "aa");
+ InternalMNode aaNode = new InternalMNode(rootNode, "aa");
rootNode.addChild(aaNode.getName(), aaNode);
// builds the relationship of root.a.b and root.aa.bb
- MNode bNode = new MNode(aNode, "b");
+ InternalMNode bNode = new InternalMNode(aNode, "b");
aNode.addChild(bNode.getName(), bNode);
- MNode bbNode = new MNode(aaNode, "bb");
+ InternalMNode bbNode = new InternalMNode(aaNode, "bb");
aaNode.addChild(bbNode.getName(), bbNode);
// builds the relationship of root.aa.bb.cc
- MNode ccNode = new MNode(bbNode, "cc");
+ InternalMNode ccNode = new InternalMNode(bbNode, "cc");
bbNode.addChild(ccNode.getName(), ccNode);
List<String> multiFullPaths = MetaUtils.getMultiFullPaths(rootNode);
diff --git a/server/src/test/java/org/apache/iotdb/db/metadata/mnode/MNodeTest.java b/server/src/test/java/org/apache/iotdb/db/metadata/mnode/MNodeTest.java
index 46494c9..7d89312 100644
--- a/server/src/test/java/org/apache/iotdb/db/metadata/mnode/MNodeTest.java
+++ b/server/src/test/java/org/apache/iotdb/db/metadata/mnode/MNodeTest.java
@@ -45,18 +45,18 @@ public class MNodeTest {
@Test
public void testReplaceChild() throws InterruptedException {
// after replacing a with c, the timeseries root.a.b becomes root.c.b
- MNode rootNode = new MNode(null, "root");
+ InternalMNode rootNode = new InternalMNode(null, "root");
- MNode aNode = new MNode(rootNode, "a");
+ InternalMNode aNode = new InternalMNode(rootNode, "a");
rootNode.addChild(aNode.getName(), aNode);
- MNode bNode = new MNode(aNode, "b");
+ InternalMNode bNode = new InternalMNode(aNode, "b");
aNode.addChild(bNode.getName(), bNode);
aNode.addAlias("aliasOfb", bNode);
for (int i = 0; i < 500; i++) {
service.submit(
- new Thread(() -> rootNode.replaceChild(aNode.getName(), new MNode(null, "c"))));
+ new Thread(() -> rootNode.replaceChild(aNode.getName(), new InternalMNode(null, "c"))));
}
if (!service.isShutdown()) {
@@ -71,28 +71,28 @@ public class MNodeTest {
@Test
public void testAddChild() {
- MNode rootNode = new MNode(null, "root");
+ InternalMNode rootNode = new InternalMNode(null, "root");
- MNode speedNode =
+ IMNode speedNode =
rootNode
- .addChild(new MNode(null, "sg1"))
- .addChild(new MNode(null, "a"))
- .addChild(new MNode(null, "b"))
- .addChild(new MNode(null, "c"))
- .addChild(new MNode(null, "d"))
- .addChild(new MNode(null, "device"))
- .addChild(new MNode(null, "speed"));
+ .addChild(new InternalMNode(null, "sg1"))
+ .addChild(new InternalMNode(null, "a"))
+ .addChild(new InternalMNode(null, "b"))
+ .addChild(new InternalMNode(null, "c"))
+ .addChild(new InternalMNode(null, "d"))
+ .addChild(new InternalMNode(null, "device"))
+ .addChild(new InternalMNode(null, "speed"));
assertEquals("root.sg1.a.b.c.d.device.speed", speedNode.getFullPath());
- MNode temperatureNode =
+ IMNode temperatureNode =
rootNode
.getChild("sg1")
- .addChild(new MNode(null, "aa"))
- .addChild(new MNode(null, "bb"))
- .addChild(new MNode(null, "cc"))
- .addChild(new MNode(null, "dd"))
- .addChild(new MNode(null, "device11"))
- .addChild(new MNode(null, "temperature"));
+ .addChild(new InternalMNode(null, "aa"))
+ .addChild(new InternalMNode(null, "bb"))
+ .addChild(new InternalMNode(null, "cc"))
+ .addChild(new InternalMNode(null, "dd"))
+ .addChild(new InternalMNode(null, "device11"))
+ .addChild(new InternalMNode(null, "temperature"));
assertEquals("root.sg1.aa.bb.cc.dd.device11.temperature", temperatureNode.getFullPath());
}
}
diff --git a/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/IMeasurementSchema.java b/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/IMeasurementSchema.java
index fb3358d..b689a12 100644
--- a/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/IMeasurementSchema.java
+++ b/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/IMeasurementSchema.java
@@ -59,6 +59,11 @@ public interface IMeasurementSchema {
int getMeasurementIdColumnIndex(String measurementId);
+ int getMeasurementCount();
+
+ /* test whether the schema contains Measurement with given measurementId */
+ boolean isCompatible(String measurementId);
+
int serializeTo(ByteBuffer buffer);
int serializeTo(OutputStream outputStream) throws IOException;
diff --git a/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/MeasurementSchema.java b/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/MeasurementSchema.java
index 4778ca6..48dc710 100644
--- a/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/MeasurementSchema.java
+++ b/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/MeasurementSchema.java
@@ -394,6 +394,16 @@ public class MeasurementSchema
@Override
public int getMeasurementIdColumnIndex(String measurementId) {
- return 0;
+ return this.measurementId.equals(measurementId) ? 0 : -1;
+ }
+
+ @Override
+ public int getMeasurementCount() {
+ return 1;
+ }
+
+ @Override
+ public boolean isCompatible(String measurementId) {
+ return this.measurementId.equals(measurementId);
}
}
diff --git a/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/VectorMeasurementSchema.java b/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/VectorMeasurementSchema.java
index f9eb2b6..9808978 100644
--- a/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/VectorMeasurementSchema.java
+++ b/tsfile/src/main/java/org/apache/iotdb/tsfile/write/schema/VectorMeasurementSchema.java
@@ -203,7 +203,17 @@ public class VectorMeasurementSchema
@Override
public int getMeasurementIdColumnIndex(String measurementId) {
- return measurementsToIndexMap.get(measurementId);
+ return measurementsToIndexMap.getOrDefault(measurementId, -1);
+ }
+
+ @Override
+ public int getMeasurementCount() {
+ return measurementsToIndexMap.size();
+ }
+
+ @Override
+ public boolean isCompatible(String measurementId) {
+ return measurementsToIndexMap.containsKey(measurementId);
}
@Override
diff --git a/tsfile/src/test/java/org/apache/iotdb/tsfile/write/writer/VectorMeasurementSchemaStub.java b/tsfile/src/test/java/org/apache/iotdb/tsfile/write/writer/VectorMeasurementSchemaStub.java
index 0307fb4..de9c4a0 100644
--- a/tsfile/src/test/java/org/apache/iotdb/tsfile/write/writer/VectorMeasurementSchemaStub.java
+++ b/tsfile/src/test/java/org/apache/iotdb/tsfile/write/writer/VectorMeasurementSchemaStub.java
@@ -125,4 +125,14 @@ public class VectorMeasurementSchemaStub implements IMeasurementSchema {
public int getMeasurementIdColumnIndex(String measurementId) {
return 0;
}
+
+ @Override
+ public int getMeasurementCount() {
+ return 0;
+ }
+
+ @Override
+ public boolean isCompatible(String measurementId) {
+ return false;
+ }
}