You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by GitBox <gi...@apache.org> on 2020/08/03 19:34:22 UTC

[GitHub] [incubator-pinot] npawar opened a new pull request #5793: Tiered storage

npawar opened a new pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793


   ## Description
   Issue: https://github.com/apache/incubator-pinot/issues/5553
   Tiered storage support in Pinot - Phase 1.
   This phase supports default tag based instance assignments only, which are not persisted in zk. 
   
   Phase 2 will introduce instanceAssignmentConfig for tiers, which will allow us to support replica groups for tiers and also let us persist the InstancePartitions
   
   Example:
   This example show how to configure segments older than 15 days move to `tier_b_OFFLINE` and segments older than 7 days move to `tier_a_OFFLINE`
   ```
   {
     "tableName": "myTable",
     "tableType": ...,
     ...
     "tierConfigs": [{
       "name": "tierA",
       "segmentSelectorType": "timeBased",
       "segmentAge": "7d",
       "storageType": "pinotServer",
       "serverTag": "tier_a_OFFLINE"
     }, {
       "name": "tierB",
       "segmentSelectorType": "timeBased",
       "segmentAge": "15d",
       "storageType": "pinotServer",
       "serverTag": "tier_b_OFFLINE"
     }] 
   }
   ```
   
   ## Release Notes
   Tiered storage phase 1 - default tag based instance assignment for tiers
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] fx19880617 commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
fx19880617 commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467517563



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/PinotServerTierStorage.java
##########
@@ -0,0 +1,46 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.pinot.common.tier.TierFactory.TierStorageType;
+
+
+/**
+ * Tier storage type which uses Pinot servers as storage
+ */
+public class PinotServerTierStorage implements TierStorage {
+  private final String _type = TierStorageType.PINOT_SERVER.toString();
+  private final String _tag;

Review comment:
       Shall we support multiple tags for a tier? 

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/SegmentAssignmentUtils.java
##########
@@ -326,4 +328,68 @@ static void rebalanceReplicaGroupBasedPartition(Map<String, Map<String, String>>
       return _offlineSegmentAssignment;
     }
   }
+
+  /**
+   * Takes a segment assignment adn splits them up based on which tiers the segments are eligible for. Only considers ONLINE segments.

Review comment:
       typo 

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierSegmentSelector.java
##########
@@ -0,0 +1,39 @@
+/**
+ * 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.pinot.common.tier;
+
+/**
+ * Interface for the segment selection strategy of a tier
+ */
+public interface TierSegmentSelector {
+
+  /**
+   * The type of the segment selector (e.g. TIME)
+   */
+  String getType();

Review comment:
       +1

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierFactory.java
##########
@@ -0,0 +1,70 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.helix.HelixManager;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Factory class to create and sort {@link Tier}
+ */
+public final class TierFactory {
+
+  /**
+   * Types of segmentSelectors for tiers
+   */
+  public enum TierSegmentSelectorType {
+    TIME
+  }
+
+  /**
+   * Types of storage for tiers
+   */
+  public enum TierStorageType {
+    PINOT_SERVER
+  }
+
+  private TierFactory() {
+  }
+
+  /**
+   * Constructs a {@link Tier} from the {@link TierConfig} in the table config
+   */
+  public static Tier getTier(TierConfig tierConfig, HelixManager helixManager) {
+    TierSegmentSelector segmentSelector;
+    TierStorage storageSelector;
+
+    String segmentSelectorType = tierConfig.getSegmentSelectorType();
+    if (segmentSelectorType.equalsIgnoreCase(TierSegmentSelectorType.TIME.toString())) {
+      segmentSelector = new TimeBasedTierSegmentSelector(helixManager, tierConfig.getSegmentAge());
+    } else {
+      throw new IllegalStateException("Unsupported segmentSelectorType: " + segmentSelectorType);
+    }
+
+    String storageSelectorType = tierConfig.getStorageType();

Review comment:
       using enum here also?

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TimeBasedTierSegmentSelector.java
##########
@@ -0,0 +1,85 @@
+/**
+ * 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.pinot.common.tier;
+
+import com.google.common.base.Preconditions;
+import java.util.concurrent.TimeUnit;
+import org.apache.helix.HelixManager;
+import org.apache.pinot.common.metadata.ZKMetadataProvider;
+import org.apache.pinot.common.metadata.segment.SegmentZKMetadata;
+import org.apache.pinot.common.tier.TierFactory.TierSegmentSelectorType;
+import org.apache.pinot.spi.utils.TimeUtils;
+import org.apache.pinot.spi.utils.builder.TableNameBuilder;
+
+
+/**
+ * A {@link TierSegmentSelector} strategy which selects segments for a tier based on the age of the segment
+ */
+public class TimeBasedTierSegmentSelector implements TierSegmentSelector {
+  private final String _type = TierSegmentSelectorType.TIME.toString();
+  private final long _segmentAgeMillis;
+  private final HelixManager _helixManager;
+
+  public TimeBasedTierSegmentSelector(HelixManager helixManager, String segmentAge) {
+    _segmentAgeMillis = TimeUtils.convertPeriodToMillis(segmentAge);
+    _helixManager = helixManager;
+  }
+
+  @Override
+  public String getType() {
+    return _type;

Review comment:
       just do `return TierSegmentSelectorType.TIME ` and remove `_type`?

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierFactory.java
##########
@@ -0,0 +1,70 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.helix.HelixManager;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Factory class to create and sort {@link Tier}
+ */
+public final class TierFactory {
+
+  /**
+   * Types of segmentSelectors for tiers
+   */
+  public enum TierSegmentSelectorType {
+    TIME
+  }
+
+  /**
+   * Types of storage for tiers
+   */
+  public enum TierStorageType {
+    PINOT_SERVER
+  }
+
+  private TierFactory() {
+  }
+
+  /**
+   * Constructs a {@link Tier} from the {@link TierConfig} in the table config
+   */
+  public static Tier getTier(TierConfig tierConfig, HelixManager helixManager) {
+    TierSegmentSelector segmentSelector;
+    TierStorage storageSelector;
+
+    String segmentSelectorType = tierConfig.getSegmentSelectorType();

Review comment:
       using enum?

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/PinotServerTierStorage.java
##########
@@ -0,0 +1,46 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.pinot.common.tier.TierFactory.TierStorageType;
+
+
+/**
+ * Tier storage type which uses Pinot servers as storage
+ */
+public class PinotServerTierStorage implements TierStorage {
+  private final String _type = TierStorageType.PINOT_SERVER.toString();

Review comment:
       We can skip this field and directly return enum in the `getType()` method.

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierStorage.java
##########
@@ -0,0 +1,30 @@
+/**
+ * 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.pinot.common.tier;
+
+/**
+ * Interface for the storage type of the tier
+ */
+public interface TierStorage {
+
+  /**
+   * Returns the type of the storage (e.g. PINOT_SERVER)
+   */
+  String getType();

Review comment:
       enum?

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierSegmentSelector.java
##########
@@ -0,0 +1,39 @@
+/**
+ * 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.pinot.common.tier;
+
+/**
+ * Interface for the segment selection strategy of a tier
+ */
+public interface TierSegmentSelector {

Review comment:
       thinking of adding a method like `int getPriority()`?
   For time based tiers we can use internal age for comparison if priority is the same.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467503882



##########
File path: pinot-spi/src/main/java/org/apache/pinot/spi/config/table/TierConfig.java
##########
@@ -0,0 +1,84 @@
+/**
+ * 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.pinot.spi.config.table;
+
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.annotation.JsonPropertyDescription;
+import com.google.common.base.Preconditions;
+import javax.annotation.Nullable;
+import org.apache.pinot.spi.config.BaseJsonConfig;
+
+
+/**
+ * Config for the tiered storage and the segments which will move to that tier
+ */
+public class TierConfig extends BaseJsonConfig {
+  @JsonPropertyDescription("Name of the tier with format TIER<number>")
+  private final String _name;
+
+  @JsonPropertyDescription("The strategy for selecting segments")
+  private final String _segmentSelectorType;
+
+  @JsonPropertyDescription("For 'TIME' segment selector, the period after which to select segments for this tier")
+  private final String _segmentAge;
+
+  @JsonPropertyDescription("The type of storage storage")
+  private final String _storageType;
+
+  @JsonPropertyDescription("For 'PINOT_SERVER' storageSelector, the tag with which to identify servers for this tier.")
+  private final String _serverTag;
+
+  // TODO: only "serverTag" is supported currently. In next iteration, "InstanceAssignmentConfig _instanceAssignmentConfig" will be added here
+
+  public TierConfig(@JsonProperty(value = "name", required = true) String name,
+      @JsonProperty(value = "segmentSelectorType", required = true) String segmentSelectorType,

Review comment:
       Enum instead of String?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#issuecomment-671099209


   > > > Made one small change not part of the review. Converted the "segmentSelectorType" and "storageType" to enums. If you want to look again.
   > > 
   > > 
   > > Since we introduced the enum, try to use enum over string for these 2 fields
   > 
   > The reason I kept it string is that if we want people to plug in their own strategies, they can do so without needing to add it to the enum
   
   In that case, IMO we should not introduce the enum


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r466742688



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/tier/TieredStorageRelocator.java
##########
@@ -0,0 +1,101 @@
+/**
+ * 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.pinot.controller.helix.core.tier;
+
+import com.google.common.base.Preconditions;
+import java.util.concurrent.ExecutorService;
+import org.apache.commons.configuration.BaseConfiguration;
+import org.apache.commons.configuration.Configuration;
+import org.apache.pinot.common.assignment.InstanceAssignmentConfigUtils;
+import org.apache.pinot.common.metrics.ControllerMetrics;
+import org.apache.pinot.controller.ControllerConf;
+import org.apache.pinot.controller.LeadControllerManager;
+import org.apache.pinot.controller.helix.core.PinotHelixResourceManager;
+import org.apache.pinot.controller.helix.core.periodictask.ControllerPeriodicTask;
+import org.apache.pinot.controller.helix.core.rebalance.RebalanceConfigConstants;
+import org.apache.pinot.controller.helix.core.rebalance.RebalanceResult;
+import org.apache.pinot.controller.helix.core.rebalance.TableRebalancer;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.stream.StreamConfig;
+import org.apache.pinot.spi.utils.TimeUtils;
+import org.apache.pinot.spi.utils.builder.TableNameBuilder;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+
+/**
+ * Periodic task to run rebalancer in background to relocate segments to storage tiers
+ * TODO: we could potentially get rid of tagOverrideConfig and rely on this relocator for moving COMPLETED segments
+ */
+public class TieredStorageRelocator extends ControllerPeriodicTask<Void> {

Review comment:
       Removed TieredStorageRelocator. Rebranded RealtimeSegmentRelocator as `SegmentRelocator`. 




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467323399



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/utils/config/TierConfigUtils.java
##########
@@ -0,0 +1,56 @@
+/**
+ * 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.pinot.common.utils.config;
+
+import java.util.ArrayList;
+import java.util.List;
+import org.apache.commons.collections.CollectionUtils;
+import org.apache.helix.HelixManager;
+import org.apache.pinot.common.tier.Tier;
+import org.apache.pinot.common.tier.TierFactory;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Util methods for TierConfig
+ */
+public class TierConfigUtils {
+
+  /**
+   * Returns whether relocation of segments to tiers has been enabled for this table
+   */
+  public static boolean shouldRelocateToTiers(TableConfig tableConfig) {
+    return CollectionUtils.isNotEmpty(tableConfig.getTierConfigsList());
+  }
+
+  /**
+   * Gets tiers for given storage type from provided list of TierConfig
+   */
+  public static List<Tier> getTiersForStorageType(List<TierConfig> tierConfigList, String storageType,

Review comment:
       sure, done




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#issuecomment-670795637


   > LGTM otherwise. One **critical** comment in `SegmentRelocator`
   
   Made one small change not part of the review. Converted the "segmentSelectorType" and "storageType" to enums. If you want to look again. 


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467503946



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/utils/config/TierConfigUtils.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.pinot.common.utils.config;
+
+import com.google.common.base.Preconditions;
+import java.util.ArrayList;
+import java.util.Comparator;
+import java.util.List;
+import org.apache.commons.collections.CollectionUtils;
+import org.apache.helix.HelixManager;
+import org.apache.pinot.common.tier.Tier;
+import org.apache.pinot.common.tier.TierFactory;
+import org.apache.pinot.common.tier.TierFactory.TierSegmentSelectorType;
+import org.apache.pinot.common.tier.TierSegmentSelector;
+import org.apache.pinot.common.tier.TierStorage;
+import org.apache.pinot.common.tier.TimeBasedTierSegmentSelector;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Util methods for TierConfig
+ */
+public final class TierConfigUtils {
+
+  private TierConfigUtils() {
+  }
+
+  /**
+   * Returns whether relocation of segments to tiers has been enabled for this table
+   */
+  public static boolean shouldRelocateToTiers(TableConfig tableConfig) {
+    return CollectionUtils.isNotEmpty(tableConfig.getTierConfigsList());
+  }
+
+  /**
+   * Gets sorted list of tiers for given storage type from provided list of TierConfig
+   */
+  public static List<Tier> getSortedTiersForStorageType(List<TierConfig> tierConfigList, String storageType,

Review comment:
       Use enum `storageType`




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#issuecomment-671167139


   > > > > Made one small change not part of the review. Converted the "segmentSelectorType" and "storageType" to enums. If you want to look again.
   > > > 
   > > > 
   > > > Since we introduced the enum, try to use enum over string for these 2 fields
   > > 
   > > 
   > > The reason I kept it string is that if we want people to plug in their own strategies, they can do so without needing to add it to the enum
   > 
   > In that case, IMO we should not introduce the enum
   
   Hmm, agreed. Reverted enum change.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#issuecomment-670999241


   > > Made one small change not part of the review. Converted the "segmentSelectorType" and "storageType" to enums. If you want to look again.
   > 
   > Since we introduced the enum, try to use enum over string for these 2 fields
   
   The reason I kept it string is that if we want people to plug in their own strategies, they can do so without needing to add it to the enum


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r466742018



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/assignment/InstanceAssignmentConfigUtils.java
##########
@@ -50,6 +50,13 @@ public static boolean shouldRelocateCompletedSegments(TableConfig tableConfig) {
         .isRelocateCompletedSegments(tableConfig.getTenantConfig());
   }
 
+  /**
+   * Returns whether relocation of segments to tiers has been enabled for this table
+   */
+  public static boolean shouldRelocateToTiers(TableConfig tableConfig) {

Review comment:
       Done




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467693955



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/PinotServerTierStorage.java
##########
@@ -0,0 +1,46 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.pinot.common.tier.TierFactory.TierStorageType;
+
+
+/**
+ * Tier storage type which uses Pinot servers as storage
+ */
+public class PinotServerTierStorage implements TierStorage {
+  private final String _type = TierStorageType.PINOT_SERVER.toString();
+  private final String _tag;

Review comment:
       Yes that's a good idea. I can add that for phase 2 (where I'll be handling advanced instance assignments for tiers)




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#issuecomment-670973573


   > Made one small change not part of the review. Converted the "segmentSelectorType" and "storageType" to enums. If you want to look again.
   
   Since we introduced the enum, try to use enum over string for these 2 fields


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r466068162



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -161,23 +185,80 @@ private void checkReplication(InstancePartitions instancePartitions) {
 
   @Override
   public Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config) {
-    InstancePartitions instancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
-    Preconditions.checkState(instancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
-        _offlineTableName);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, Configuration config) {
+    InstancePartitions offlineInstancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
+    Preconditions
+        .checkState(offlineInstancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
+            _offlineTableName);
     boolean bootstrap =
         config.getBoolean(RebalanceConfigConstants.BOOTSTRAP, RebalanceConfigConstants.DEFAULT_BOOTSTRAP);
+
+    Map<String, Map<String, String>> subsetAssignment = currentAssignment;
+    // rebalance tiers first
+    List<Map<String, Map<String, String>>> newTierAssignments = null;
+    if (tierInstancePartitionsMap != null && !tierInstancePartitionsMap.isEmpty()) {

Review comment:
       Use `InstanceAssignmentConfigUtils.shouldRelocateToTiers(tableConfig)` instead of checking map to determine whether to relocate tiers

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/assignment/InstanceAssignmentConfigUtils.java
##########
@@ -50,6 +50,13 @@ public static boolean shouldRelocateCompletedSegments(TableConfig tableConfig) {
         .isRelocateCompletedSegments(tableConfig.getTenantConfig());
   }
 
+  /**
+   * Returns whether relocation of segments to tiers has been enabled for this table
+   */
+  public static boolean shouldRelocateToTiers(TableConfig tableConfig) {
+    return tableConfig.getTierConfigsList() != null && !tableConfig.getTierConfigsList().isEmpty();

Review comment:
       (nit)
   ```suggestion
       return CollectionUtils.isNotEmpty(tableConfig.getTierConfigsList());
   ```

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/assignment/InstanceAssignmentConfigUtils.java
##########
@@ -50,6 +50,13 @@ public static boolean shouldRelocateCompletedSegments(TableConfig tableConfig) {
         .isRelocateCompletedSegments(tableConfig.getTenantConfig());
   }
 
+  /**
+   * Returns whether relocation of segments to tiers has been enabled for this table
+   */
+  public static boolean shouldRelocateToTiers(TableConfig tableConfig) {

Review comment:
       Consider moving this into a separate class `TierConfigUtils`?

##########
File path: pinot-spi/src/main/java/org/apache/pinot/spi/config/table/TierConfig.java
##########
@@ -0,0 +1,80 @@
+/**
+ * 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.pinot.spi.config.table;
+
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.annotation.JsonPropertyDescription;
+import javax.annotation.Nullable;
+import org.apache.pinot.spi.config.BaseJsonConfig;
+
+
+/**
+ * Config for the tiered storage and the segments which will move to that tier
+ */
+public class TierConfig extends BaseJsonConfig {
+  @JsonPropertyDescription("Name of the tier with format TIER<number>")
+  private final String _name;
+
+  @JsonPropertyDescription("The strategy for selecting segments")
+  private final String _segmentSelectorType;
+
+  @JsonPropertyDescription("For 'timeBased' segment selector, the period after which to select segments for this tier")
+  private final String _segmentAge;
+
+  @JsonPropertyDescription("The type of storage storage")
+  private final String _storageType;
+
+  @JsonPropertyDescription("For 'pinotServer' storageSelector, the tag with which to identify servers for this tier.")
+  private final String _serverTag;
+
+  // TODO: only "serverTag" is supported currently. In next iteration, "InstanceAssignmentConfig _instanceAssignmentConfig" will be added here
+
+  public TierConfig(@JsonProperty(value = "name", required = true) String name,
+      @JsonProperty(value = "segmentSelectorType", required = true) String segmentSelectorType,
+      @JsonProperty("segmentAge") @Nullable String segmentAge,
+      @JsonProperty(value = "storageType", required = true) String storageType,
+      @JsonProperty("serverTag") @Nullable String serverTag) {
+    _name = name;

Review comment:
       `Preconditions.checkArgument()` on all non-null arguments to prevent bad config

##########
File path: pinot-core/src/main/java/org/apache/pinot/core/util/TableConfigUtils.java
##########
@@ -136,9 +143,55 @@ private static void validateIngestionConfig(@Nullable IngestionConfig ingestionC
         }
         // TODO: remove this once we add support for derived columns/chained transform functions
         if (!Collections.disjoint(transformColumns, argumentColumns)) {
-          throw new IllegalStateException("Derived columns not supported yet. Cannot use a transform column as argument to another transform functions");
+          throw new IllegalStateException(
+              "Derived columns not supported yet. Cannot use a transform column as argument to another transform functions");
         }
       }
     }
   }
+
+  /**
+   * Validates the tier configs
+   * Checks for the right segmentSelectorType and its required properties
+   * Checks for the right storageType and its required properties
+   */
+  private static void validateTierConfigList(@Nullable List<TierConfig> tierConfigList) {
+    if (tierConfigList == null) {
+      return;
+    }
+
+    Set<String> tierNames = new HashSet<>();
+    for (TierConfig tierConfig : tierConfigList) {
+      String tierName = tierConfig.getName();
+      Preconditions.checkState(!tierName.isEmpty());
+      Preconditions.checkState(!tierNames.contains(tierName), "Tier name: %s already exists in tier configs", tierName);
+      tierNames.add(tierName);

Review comment:
       (nit)
   ```suggestion
         Preconditions.checkState(tierNames.add(tierName), "Tier name: %s already exists in tier configs", tierName);
   ```

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -82,6 +91,10 @@ public void init(HelixManager helixManager, TableConfig tableConfig) {
         tableConfig.getValidationConfig().getReplicaGroupStrategyConfig();
     _partitionColumn = replicaGroupStrategyConfig != null ? replicaGroupStrategyConfig.getPartitionColumn() : null;
 
+    if (InstanceAssignmentConfigUtils.shouldRelocateToTiers(tableConfig)) {

Review comment:
       Move this calculation into the `rebalanceTable()` because we don't need it for `assignSegment()`. Same for `RealtimeSegmentAssignment`

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -161,23 +185,80 @@ private void checkReplication(InstancePartitions instancePartitions) {
 
   @Override
   public Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config) {
-    InstancePartitions instancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
-    Preconditions.checkState(instancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
-        _offlineTableName);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, Configuration config) {
+    InstancePartitions offlineInstancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
+    Preconditions
+        .checkState(offlineInstancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
+            _offlineTableName);
     boolean bootstrap =
         config.getBoolean(RebalanceConfigConstants.BOOTSTRAP, RebalanceConfigConstants.DEFAULT_BOOTSTRAP);
+
+    Map<String, Map<String, String>> subsetAssignment = currentAssignment;
+    // rebalance tiers first
+    List<Map<String, Map<String, String>>> newTierAssignments = null;
+    if (tierInstancePartitionsMap != null && !tierInstancePartitionsMap.isEmpty()) {
+      LOGGER.info("Rebalancing tiers: {} for table: {} with bootstrap: {}", tierInstancePartitionsMap.keySet(),
+          _offlineTableName, bootstrap);
+
+      // get tier to segment assignment map i.e. current assignments split by tiers they are eligible for
+      SegmentAssignmentUtils.TierSegmentAssignment tierSegmentAssignment =
+          new SegmentAssignmentUtils.TierSegmentAssignment(_offlineTableName, _sortedTiers, currentAssignment);
+      Map<String, Map<String, Map<String, String>>> tierNameToSegmentAssignmentMap =
+          tierSegmentAssignment.getTierNameToSegmentAssignmentMap();
+
+      // for each tier, calculate new assignment using instancePartitions for that tier
+      newTierAssignments = new ArrayList<>(tierNameToSegmentAssignmentMap.size());
+      for (Map.Entry<String, Map<String, Map<String, String>>> entry : tierNameToSegmentAssignmentMap.entrySet()) {
+        String tierName = entry.getKey();
+        Map<String, Map<String, String>> tierCurrentAssignment = entry.getValue();
+
+        InstancePartitions tierInstancePartitions = tierInstancePartitionsMap.get(tierName);
+        Preconditions
+            .checkNotNull(tierInstancePartitions, "Failed to find instance partitions for tier: %s of table: %s",
+                tierName, _offlineTableName);
+        checkReplication(tierInstancePartitions);
+
+        LOGGER.info("Rebalancing tier: {} for table: {} with instance partitions: {}", tierName, _offlineTableName,
+            tierInstancePartitions);
+        newTierAssignments.add(reassignSegments(tierName, tierCurrentAssignment, tierInstancePartitions, bootstrap));
+      }
+
+      // rest of the operations should happen only on segments which were not already assigned as part of tiers
+      subsetAssignment = tierSegmentAssignment.getNonTierSegmentAssignment();
+    }
+
     LOGGER.info("Rebalancing table: {} with instance partitions: {}, bootstrap: {}", _offlineTableName,
-        instancePartitions, bootstrap);
-    checkReplication(instancePartitions);
+        offlineInstancePartitions, bootstrap);
+    checkReplication(offlineInstancePartitions);
+    Map<String, Map<String, String>> newAssignment =
+        reassignSegments(InstancePartitionsType.OFFLINE.toString(), subsetAssignment, offlineInstancePartitions,
+            bootstrap);
+
+    // add tier assignments, if available
+    if (CollectionUtils.isNotEmpty(newTierAssignments)) {
+      newTierAssignments.forEach(newAssignment::putAll);
+    }
+
+    LOGGER.info("Rebalanced table: {}, number of segments to be moved to each instance: {}", _offlineTableName,
+        SegmentAssignmentUtils.getNumSegmentsToBeMovedPerInstance(currentAssignment, newAssignment));
+    return newAssignment;
+  }
 
+  /**
+   * Rebalances segments in the current assignment using the instancePartitions and returns new assignment
+   */
+  private Map<String, Map<String, String>> reassignSegments(String instancePartitionType,
+      Map<String, Map<String, String>> currentSegmentAssignment, InstancePartitions instancePartitions,

Review comment:
       (nit) `currentAssignment` for concise?

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/tier/TieredStorageRelocator.java
##########
@@ -0,0 +1,101 @@
+/**
+ * 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.pinot.controller.helix.core.tier;
+
+import com.google.common.base.Preconditions;
+import java.util.concurrent.ExecutorService;
+import org.apache.commons.configuration.BaseConfiguration;
+import org.apache.commons.configuration.Configuration;
+import org.apache.pinot.common.assignment.InstanceAssignmentConfigUtils;
+import org.apache.pinot.common.metrics.ControllerMetrics;
+import org.apache.pinot.controller.ControllerConf;
+import org.apache.pinot.controller.LeadControllerManager;
+import org.apache.pinot.controller.helix.core.PinotHelixResourceManager;
+import org.apache.pinot.controller.helix.core.periodictask.ControllerPeriodicTask;
+import org.apache.pinot.controller.helix.core.rebalance.RebalanceConfigConstants;
+import org.apache.pinot.controller.helix.core.rebalance.RebalanceResult;
+import org.apache.pinot.controller.helix.core.rebalance.TableRebalancer;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.stream.StreamConfig;
+import org.apache.pinot.spi.utils.TimeUtils;
+import org.apache.pinot.spi.utils.builder.TableNameBuilder;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+
+/**
+ * Periodic task to run rebalancer in background to relocate segments to storage tiers
+ * TODO: we could potentially get rid of tagOverrideConfig and rely on this relocator for moving COMPLETED segments
+ */
+public class TieredStorageRelocator extends ControllerPeriodicTask<Void> {

Review comment:
       Please keep only one segment relocator. Currently there are 2 relocators: `TieredStorageRelocator` and `RealtimeSegmentRelocator`. They can rebalance the same table at the same time which could cause problem. Recommend replacing `RealtimeSegmentRelocator` with `SegmentRelocator` which handles both completed segment relocation and tier storage relocation.

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -161,23 +185,80 @@ private void checkReplication(InstancePartitions instancePartitions) {
 
   @Override
   public Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config) {
-    InstancePartitions instancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
-    Preconditions.checkState(instancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
-        _offlineTableName);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, Configuration config) {
+    InstancePartitions offlineInstancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
+    Preconditions
+        .checkState(offlineInstancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
+            _offlineTableName);
     boolean bootstrap =
         config.getBoolean(RebalanceConfigConstants.BOOTSTRAP, RebalanceConfigConstants.DEFAULT_BOOTSTRAP);
+
+    Map<String, Map<String, String>> subsetAssignment = currentAssignment;
+    // rebalance tiers first
+    List<Map<String, Map<String, String>>> newTierAssignments = null;
+    if (tierInstancePartitionsMap != null && !tierInstancePartitionsMap.isEmpty()) {
+      LOGGER.info("Rebalancing tiers: {} for table: {} with bootstrap: {}", tierInstancePartitionsMap.keySet(),
+          _offlineTableName, bootstrap);
+
+      // get tier to segment assignment map i.e. current assignments split by tiers they are eligible for
+      SegmentAssignmentUtils.TierSegmentAssignment tierSegmentAssignment =
+          new SegmentAssignmentUtils.TierSegmentAssignment(_offlineTableName, _sortedTiers, currentAssignment);
+      Map<String, Map<String, Map<String, String>>> tierNameToSegmentAssignmentMap =
+          tierSegmentAssignment.getTierNameToSegmentAssignmentMap();
+
+      // for each tier, calculate new assignment using instancePartitions for that tier
+      newTierAssignments = new ArrayList<>(tierNameToSegmentAssignmentMap.size());
+      for (Map.Entry<String, Map<String, Map<String, String>>> entry : tierNameToSegmentAssignmentMap.entrySet()) {
+        String tierName = entry.getKey();
+        Map<String, Map<String, String>> tierCurrentAssignment = entry.getValue();
+
+        InstancePartitions tierInstancePartitions = tierInstancePartitionsMap.get(tierName);
+        Preconditions
+            .checkNotNull(tierInstancePartitions, "Failed to find instance partitions for tier: %s of table: %s",
+                tierName, _offlineTableName);
+        checkReplication(tierInstancePartitions);
+
+        LOGGER.info("Rebalancing tier: {} for table: {} with instance partitions: {}", tierName, _offlineTableName,

Review comment:
       (nit) Including `bootstrap` in the log?

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -91,6 +104,17 @@ public void init(HelixManager helixManager, TableConfig tableConfig) {
     }
   }
 
+  /**
+   * Returns a sorted list of Tiers from the TierConfigList in table config.
+   * Keeps only those which have "pinotServer" storage type.
+   */
+  @VisibleForTesting
+  protected List<Tier> getSortedTiersForPinotServerStorage(List<TierConfig> tierConfigList) {
+    return tierConfigList.stream().filter(t -> TierFactory.PINOT_SERVER_STORAGE_TYPE.equals(t.getStorageType()))

Review comment:
       Prefer the old non-lambda way for both performance and readability.
   Also consider moving this common logic into `TierUtils`

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -161,23 +185,80 @@ private void checkReplication(InstancePartitions instancePartitions) {
 
   @Override
   public Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config) {
-    InstancePartitions instancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
-    Preconditions.checkState(instancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
-        _offlineTableName);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, Configuration config) {
+    InstancePartitions offlineInstancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
+    Preconditions
+        .checkState(offlineInstancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
+            _offlineTableName);
     boolean bootstrap =
         config.getBoolean(RebalanceConfigConstants.BOOTSTRAP, RebalanceConfigConstants.DEFAULT_BOOTSTRAP);
+
+    Map<String, Map<String, String>> subsetAssignment = currentAssignment;

Review comment:
       Rename to `nonTierAssignment` for clarity?

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -161,23 +185,80 @@ private void checkReplication(InstancePartitions instancePartitions) {
 
   @Override
   public Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config) {
-    InstancePartitions instancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
-    Preconditions.checkState(instancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
-        _offlineTableName);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, Configuration config) {
+    InstancePartitions offlineInstancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
+    Preconditions
+        .checkState(offlineInstancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
+            _offlineTableName);
     boolean bootstrap =
         config.getBoolean(RebalanceConfigConstants.BOOTSTRAP, RebalanceConfigConstants.DEFAULT_BOOTSTRAP);
+
+    Map<String, Map<String, String>> subsetAssignment = currentAssignment;
+    // rebalance tiers first

Review comment:
       (nit) Capitalize the first letter for convention




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r466742315



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -91,6 +104,17 @@ public void init(HelixManager helixManager, TableConfig tableConfig) {
     }
   }
 
+  /**
+   * Returns a sorted list of Tiers from the TierConfigList in table config.
+   * Keeps only those which have "pinotServer" storage type.
+   */
+  @VisibleForTesting
+  protected List<Tier> getSortedTiersForPinotServerStorage(List<TierConfig> tierConfigList) {
+    return tierConfigList.stream().filter(t -> TierFactory.PINOT_SERVER_STORAGE_TYPE.equals(t.getStorageType()))

Review comment:
       Moved to TierConfigUtils, and removed lambdas




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#issuecomment-670244811


   > Well done.
   > Most comments are minor (comments in `OfflineSegmentAssignment` also applies to `RealtimeSegmentAssignment`).
   > Please merge the 2 relocators into one.
   
   Thanks for the review. I've merged the 2 relocators into `SegmentRelocator`. I introduced controller confs for `controller.segment.relocator.frequency`, and deprecated the confs for `.controller.realtime.segment.relocator.frequency`. For some releases, we'll carry both around.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar merged pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar merged pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793


   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467195264



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/ControllerStarter.java
##########
@@ -688,7 +684,7 @@ public static ControllerStarter startDefault(File webappPath) {
     conf.setRealtimeSegmentValidationFrequencyInSeconds(3600);
     conf.setBrokerResourceValidationFrequencyInSeconds(3600);
     conf.setStatusCheckerFrequencyInSeconds(5 * 60);
-    conf.setRealtimeSegmentRelocatorFrequency("1h");
+    conf.setSegmentRelocatorFrequencyInSeconds(60*60);

Review comment:
       (nit) reformat (or just put 3600)

##########
File path: pinot-common/pom.xml
##########
@@ -182,6 +182,10 @@
       <groupId>org.apache.commons</groupId>
       <artifactId>commons-compress</artifactId>
     </dependency>
+    <dependency>

Review comment:
       This is already included in the `pinot-spi`, no need to include again (you might need to rebase to the current master)

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/utils/config/TierConfigUtils.java
##########
@@ -0,0 +1,56 @@
+/**
+ * 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.pinot.common.utils.config;
+
+import java.util.ArrayList;
+import java.util.List;
+import org.apache.commons.collections.CollectionUtils;
+import org.apache.helix.HelixManager;
+import org.apache.pinot.common.tier.Tier;
+import org.apache.pinot.common.tier.TierFactory;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Util methods for TierConfig
+ */
+public class TierConfigUtils {

Review comment:
       Add a private constructor

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierFactory.java
##########
@@ -0,0 +1,83 @@
+/**
+ * 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.pinot.common.tier;
+
+import com.google.common.base.Preconditions;
+import java.util.Comparator;
+import org.apache.helix.HelixManager;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Factory class to create and sort {@link Tier}
+ */
+public final class TierFactory {
+
+  public static final String TIME_BASED_SEGMENT_SELECTOR_TYPE = "timeBased";
+  public static final String PINOT_SERVER_STORAGE_TYPE = "pinotServer";
+
+  private TierFactory() {
+  }
+
+  /**
+   * Constructs a {@link Tier} from the {@link TierConfig} in the table config
+   */
+  public static Tier getTier(TierConfig tierConfig, HelixManager helixManager) {
+    TierSegmentSelector segmentSelector;
+    TierStorage storageSelector;
+
+    String segmentSelectorType = tierConfig.getSegmentSelectorType();
+    if (segmentSelectorType.equalsIgnoreCase(TIME_BASED_SEGMENT_SELECTOR_TYPE)) {
+      segmentSelector = new TimeBasedTierSegmentSelector(helixManager, tierConfig.getSegmentAge());
+    } else {
+      throw new IllegalStateException("Unsupported segmentSelectorType: " + segmentSelectorType);
+    }
+
+    String storageSelectorType = tierConfig.getStorageType();
+    if (storageSelectorType.equalsIgnoreCase(PINOT_SERVER_STORAGE_TYPE)) {
+      storageSelector = new PinotServerTierStorage(tierConfig.getServerTag());
+    } else {
+      throw new IllegalStateException("Unsupported storageType: " + storageSelectorType);
+    }
+
+    return new Tier(tierConfig.getName(), segmentSelector, storageSelector);
+  }
+
+  /**
+   * Comparator for sorting the {@link Tier}.
+   * As of now, we have only 1 type of {@link TierSegmentSelector} and 1 type of {@link TierStorage}.
+   * Tier with an older age bucket in {@link TimeBasedTierSegmentSelector} should appear before a younger age bucket, in sort order
+   * TODO: As we add more types, this logic needs to be upgraded
+   */
+  public static Comparator<Tier> getTierComparator() {

Review comment:
       Move this into `TierConfigUtils`?

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/ControllerConf.java
##########
@@ -460,6 +459,30 @@ public void setStatusCheckerWaitForPushTimeInSeconds(int statusCheckerWaitForPus
         Integer.toString(statusCheckerWaitForPushTimeInSeconds));
   }
 
+  /**
+   * RealtimeSegmentRelocator has been rebranded to SegmentRelocator.
+   * Check for SEGMENT_RELOCATOR_FREQUENCY_IN_SECONDS property, if not found, return REALTIME_SEGMENT_RELOCATOR_FREQUENCY
+   */
+  public int getSegmentRelocatorFrequencyInSeconds() {

Review comment:
       Well done

##########
File path: pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/ControllerPeriodicTasksIntegrationTest.java
##########
@@ -111,8 +111,8 @@ public void setUp()
     properties.put(ControllerConf.CLUSTER_TENANT_ISOLATION_ENABLE, false);
     properties.put(ControllerPeriodicTasksConf.STATUS_CHECKER_INITIAL_DELAY_IN_SECONDS, PERIODIC_TASK_INITIAL_DELAY_SECONDS);
     properties.put(ControllerPeriodicTasksConf.STATUS_CHECKER_FREQUENCY_IN_SECONDS, PERIODIC_TASK_FREQUENCY_SECONDS);
-    properties.put(ControllerPeriodicTasksConf.REALTIME_SEGMENT_RELOCATION_INITIAL_DELAY_IN_SECONDS, PERIODIC_TASK_INITIAL_DELAY_SECONDS);
-    properties.put(ControllerPeriodicTasksConf.REALTIME_SEGMENT_RELOCATOR_FREQUENCY, PERIODIC_TASK_FREQUENCY);
+    properties.put(ControllerPeriodicTasksConf.DEPRECATED_REALTIME_SEGMENT_RELOCATION_INITIAL_DELAY_IN_SECONDS, PERIODIC_TASK_INITIAL_DELAY_SECONDS);

Review comment:
       Should we use `SEGMENT_RELOCATION_INITIAL_DELAY_IN_SECONDS` and `SEGMENT_RELOCATOR_FREQUENCY_IN_SECONDS`?

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -161,19 +172,78 @@ private void checkReplication(InstancePartitions instancePartitions) {
 
   @Override
   public Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config) {
-    InstancePartitions instancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
-    Preconditions.checkState(instancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
-        _offlineTableName);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, @Nullable List<Tier> sortedTiers,
+      Configuration config) {
+    InstancePartitions offlineInstancePartitions = instancePartitionsMap.get(InstancePartitionsType.OFFLINE);
+    Preconditions
+        .checkState(offlineInstancePartitions != null, "Failed to find OFFLINE instance partitions for table: %s",
+            _offlineTableName);
     boolean bootstrap =
         config.getBoolean(RebalanceConfigConstants.BOOTSTRAP, RebalanceConfigConstants.DEFAULT_BOOTSTRAP);
+
+    Map<String, Map<String, String>> nonTierAssignment = currentAssignment;
+    // Rebalance tiers first
+    List<Map<String, Map<String, String>>> newTierAssignments = null;
+    if (TierConfigUtils.shouldRelocateToTiers(_tableConfig)) {

Review comment:
       `shouldRelocateToTiers` logic is handled on the caller side, so here we can just check whether `sortedTiers` is not `null`. Same for realtime

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/SegmentAssignment.java
##########
@@ -60,9 +62,13 @@
    *
    * @param currentAssignment Current segment assignment of the table (map from segment name to instance state map)
    * @param instancePartitionsMap Map from type (OFFLINE|CONSUMING|COMPLETED) to instance partitions
+   * @param tierInstancePartitionsMap Map from tierName to instance partitions
+   * @param sortedTiers List of Tiers sorted as per priority
    * @param config Configuration for the rebalance
    * @return Rebalanced assignment for the segments
    */
   Map<String, Map<String, String>> rebalanceTable(Map<String, Map<String, String>> currentAssignment,
-      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap, Configuration config);
+      Map<InstancePartitionsType, InstancePartitions> instancePartitionsMap,
+      @Nullable Map<String, InstancePartitions> tierInstancePartitionsMap, @Nullable List<Tier> sortedTiers,

Review comment:
       (nit) Slightly more readable if we put `sortedTiers` in front of `tierInstancePartitionsMap` as `tierInstancePartitionsMap` is describing the `sortedTiers`?

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -72,10 +81,12 @@
   private String _offlineTableName;
   private int _replication;
   private String _partitionColumn;
+  private TableConfig _tableConfig;

Review comment:
       (OCD) move this to line 81

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/RealtimeSegmentAssignment.java
##########
@@ -79,18 +88,33 @@
 public class RealtimeSegmentAssignment implements SegmentAssignment {
   private static final Logger LOGGER = LoggerFactory.getLogger(RealtimeSegmentAssignment.class);
 
+  private HelixManager _helixManager;
+  private TableConfig _tableConfig;
   private String _realtimeTableName;
   private int _replication;
 
   @Override
   public void init(HelixManager helixManager, TableConfig tableConfig) {
+    _helixManager = helixManager;
+    _tableConfig = tableConfig;
     _realtimeTableName = tableConfig.getTableName();
     _replication = tableConfig.getValidationConfig().getReplicasPerPartitionNumber();
 
     LOGGER.info("Initialized RealtimeSegmentAssignment with replication: {} for table: {}", _replication,
         _realtimeTableName);
   }
 
+  /**
+   * Returns a sorted list of Tiers from the TierConfigList in table config.
+   * Keeps only those which have "pinotServer" storage type.
+   */
+  @VisibleForTesting
+  protected List<Tier> getSortedTiersForPinotServerStorage(List<TierConfig> tierConfigList) {

Review comment:
       Remove this method

##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/relocation/SegmentRelocator.java
##########
@@ -40,41 +41,51 @@
 
 
 /**
- * Periodic task to run rebalancer in background to relocate COMPLETED segments for LLC real-time table. Allow at most
- * one replica unavailable during rebalance.
+ * Periodic task to run rebalancer in background to
+ * 1. relocate COMPLETED segments to tag overrides
+ * 2. relocate ONLINE segments to tiers if tier configs are set
+ * Allow at most one replica unavailable during rebalance. Not applicable for HLC tables.
  */
-public class RealtimeSegmentRelocator extends ControllerPeriodicTask<Void> {
-  private static final Logger LOGGER = LoggerFactory.getLogger(RealtimeSegmentRelocator.class);
+public class SegmentRelocator extends ControllerPeriodicTask<Void> {
+  private static final Logger LOGGER = LoggerFactory.getLogger(SegmentRelocator.class);
 
   private final ExecutorService _executorService;
 
-  public RealtimeSegmentRelocator(PinotHelixResourceManager pinotHelixResourceManager,
+  public SegmentRelocator(PinotHelixResourceManager pinotHelixResourceManager,
       LeadControllerManager leadControllerManager, ControllerConf config, ControllerMetrics controllerMetrics,
       ExecutorService executorService) {
-    super("RealtimeSegmentRelocator", getRunFrequencySeconds(config.getRealtimeSegmentRelocatorFrequency()),
-        config.getRealtimeSegmentRelocationInitialDelayInSeconds(), pinotHelixResourceManager, leadControllerManager,
+    super(SegmentRelocator.class.getSimpleName(), 60,
+        30, pinotHelixResourceManager, leadControllerManager,

Review comment:
       **CRITICAL**
   ```suggestion
       super(SegmentRelocator.class.getSimpleName(), config.getSegmentRelocatorFrequencyInSeconds(),
           config.getSegmentRelocatorInitialDelayInSeconds(), pinotHelixResourceManager, leadControllerManager,
           controllerMetrics);
   ```

##########
File path: pinot-common/src/main/java/org/apache/pinot/common/utils/config/TierConfigUtils.java
##########
@@ -0,0 +1,56 @@
+/**
+ * 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.pinot.common.utils.config;
+
+import java.util.ArrayList;
+import java.util.List;
+import org.apache.commons.collections.CollectionUtils;
+import org.apache.helix.HelixManager;
+import org.apache.pinot.common.tier.Tier;
+import org.apache.pinot.common.tier.TierFactory;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.config.table.TierConfig;
+
+
+/**
+ * Util methods for TierConfig
+ */
+public class TierConfigUtils {
+
+  /**
+   * Returns whether relocation of segments to tiers has been enabled for this table
+   */
+  public static boolean shouldRelocateToTiers(TableConfig tableConfig) {
+    return CollectionUtils.isNotEmpty(tableConfig.getTierConfigsList());
+  }
+
+  /**
+   * Gets tiers for given storage type from provided list of TierConfig
+   */
+  public static List<Tier> getTiersForStorageType(List<TierConfig> tierConfigList, String storageType,

Review comment:
       Change this to `getSortedTiersForStorageType` and sort tiers here




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467503689



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/PinotServerTierStorage.java
##########
@@ -0,0 +1,46 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.pinot.common.tier.TierFactory.TierStorageType;
+
+
+/**
+ * Tier storage type which uses Pinot servers as storage
+ */
+public class PinotServerTierStorage implements TierStorage {
+  private final String _type = TierStorageType.PINOT_SERVER.toString();
+  private final String _tag;
+
+  public PinotServerTierStorage(String tag) {
+    _tag = tag;
+  }
+
+  /**
+   * Returns the tag used to identify the servers being used as the tier storage
+   */
+  public String getTag() {
+    return _tag;
+  }
+
+  @Override
+  public String getType() {

Review comment:
       Should we return `TierStorageType` here?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] Jackie-Jiang commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
Jackie-Jiang commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467503751



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierSegmentSelector.java
##########
@@ -0,0 +1,39 @@
+/**
+ * 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.pinot.common.tier;
+
+/**
+ * Interface for the segment selection strategy of a tier
+ */
+public interface TierSegmentSelector {
+
+  /**
+   * The type of the segment selector (e.g. TIME)
+   */
+  String getType();

Review comment:
       Return `TierSegmentSelectorType` here?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467323968



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -72,10 +81,12 @@
   private String _offlineTableName;
   private int _replication;
   private String _partitionColumn;
+  private TableConfig _tableConfig;

Review comment:
       even better, removed it because we no longer are checking `shouldRelocateToTiers(tableConfig)` in this file




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467323816



##########
File path: pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/ControllerPeriodicTasksIntegrationTest.java
##########
@@ -111,8 +111,8 @@ public void setUp()
     properties.put(ControllerConf.CLUSTER_TENANT_ISOLATION_ENABLE, false);
     properties.put(ControllerPeriodicTasksConf.STATUS_CHECKER_INITIAL_DELAY_IN_SECONDS, PERIODIC_TASK_INITIAL_DELAY_SECONDS);
     properties.put(ControllerPeriodicTasksConf.STATUS_CHECKER_FREQUENCY_IN_SECONDS, PERIODIC_TASK_FREQUENCY_SECONDS);
-    properties.put(ControllerPeriodicTasksConf.REALTIME_SEGMENT_RELOCATION_INITIAL_DELAY_IN_SECONDS, PERIODIC_TASK_INITIAL_DELAY_SECONDS);
-    properties.put(ControllerPeriodicTasksConf.REALTIME_SEGMENT_RELOCATOR_FREQUENCY, PERIODIC_TASK_FREQUENCY);
+    properties.put(ControllerPeriodicTasksConf.DEPRECATED_REALTIME_SEGMENT_RELOCATION_INITIAL_DELAY_IN_SECONDS, PERIODIC_TASK_INITIAL_DELAY_SECONDS);

Review comment:
       i kept the old ones, so that we have a test to check the old way, because people might be using the old properties. Probably after couple release can change it new?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r466742218



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/assignment/segment/OfflineSegmentAssignment.java
##########
@@ -82,6 +91,10 @@ public void init(HelixManager helixManager, TableConfig tableConfig) {
         tableConfig.getValidationConfig().getReplicaGroupStrategyConfig();
     _partitionColumn = replicaGroupStrategyConfig != null ? replicaGroupStrategyConfig.getPartitionColumn() : null;
 
+    if (InstanceAssignmentConfigUtils.shouldRelocateToTiers(tableConfig)) {

Review comment:
       Done. Added `sortedTiers` list to the rebalance method




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467528875



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/TierSegmentSelector.java
##########
@@ -0,0 +1,39 @@
+/**
+ * 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.pinot.common.tier;
+
+/**
+ * Interface for the segment selection strategy of a tier
+ */
+public interface TierSegmentSelector {

Review comment:
       I thought I'll add that when the requirements demand it.. As of now, I didn't see it being requirwd




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467693229



##########
File path: pinot-common/src/main/java/org/apache/pinot/common/tier/PinotServerTierStorage.java
##########
@@ -0,0 +1,46 @@
+/**
+ * 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.pinot.common.tier;
+
+import org.apache.pinot.common.tier.TierFactory.TierStorageType;
+
+
+/**
+ * Tier storage type which uses Pinot servers as storage
+ */
+public class PinotServerTierStorage implements TierStorage {
+  private final String _type = TierStorageType.PINOT_SERVER.toString();

Review comment:
       Removed enums and kept just strings. When we eventually support users plugging in their own strategies, enum will restrict them from doing that




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org


[GitHub] [incubator-pinot] npawar commented on a change in pull request #5793: Tiered storage

Posted by GitBox <gi...@apache.org>.
npawar commented on a change in pull request #5793:
URL: https://github.com/apache/incubator-pinot/pull/5793#discussion_r467323322



##########
File path: pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/relocation/SegmentRelocator.java
##########
@@ -40,41 +41,51 @@
 
 
 /**
- * Periodic task to run rebalancer in background to relocate COMPLETED segments for LLC real-time table. Allow at most
- * one replica unavailable during rebalance.
+ * Periodic task to run rebalancer in background to
+ * 1. relocate COMPLETED segments to tag overrides
+ * 2. relocate ONLINE segments to tiers if tier configs are set
+ * Allow at most one replica unavailable during rebalance. Not applicable for HLC tables.
  */
-public class RealtimeSegmentRelocator extends ControllerPeriodicTask<Void> {
-  private static final Logger LOGGER = LoggerFactory.getLogger(RealtimeSegmentRelocator.class);
+public class SegmentRelocator extends ControllerPeriodicTask<Void> {
+  private static final Logger LOGGER = LoggerFactory.getLogger(SegmentRelocator.class);
 
   private final ExecutorService _executorService;
 
-  public RealtimeSegmentRelocator(PinotHelixResourceManager pinotHelixResourceManager,
+  public SegmentRelocator(PinotHelixResourceManager pinotHelixResourceManager,
       LeadControllerManager leadControllerManager, ControllerConf config, ControllerMetrics controllerMetrics,
       ExecutorService executorService) {
-    super("RealtimeSegmentRelocator", getRunFrequencySeconds(config.getRealtimeSegmentRelocatorFrequency()),
-        config.getRealtimeSegmentRelocationInitialDelayInSeconds(), pinotHelixResourceManager, leadControllerManager,
+    super(SegmentRelocator.class.getSimpleName(), 60,
+        30, pinotHelixResourceManager, leadControllerManager,

Review comment:
       removed!




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org