You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@shardingsphere.apache.org by GitBox <gi...@apache.org> on 2020/04/28 08:47:45 UTC

[GitHub] [shardingsphere] tristaZero commented on a change in pull request #5363: a custom datetime shard algorithm used to shard tables by days/weeks/months/QUARTER_OF_YEAR

tristaZero commented on a change in pull request #5363:
URL: https://github.com/apache/shardingsphere/pull/5363#discussion_r416425277



##########
File path: sharding-core/sharding-core-common/src/main/java/org/apache/shardingsphere/core/strategy/algorithm/sharding/CustomDateTimeShardingAlgorithm.java
##########
@@ -0,0 +1,182 @@
+/*
+ * 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.shardingsphere.core.strategy.algorithm.sharding;
+
+import com.google.common.base.Preconditions;
+import lombok.Getter;
+import lombok.Setter;
+import org.apache.shardingsphere.api.sharding.standard.PreciseShardingValue;
+import org.apache.shardingsphere.api.sharding.standard.RangeShardingValue;
+import org.apache.shardingsphere.api.sharding.standard.StandardShardingAlgorithm;
+
+import java.time.LocalDateTime;
+import java.time.format.DateTimeFormatter;
+import java.time.format.DateTimeParseException;
+import java.time.temporal.ChronoField;
+import java.time.temporal.ChronoUnit;
+import java.time.temporal.IsoFields;
+import java.time.temporal.TemporalField;
+import java.util.Collection;
+import java.util.HashSet;
+import java.util.Properties;
+import java.util.Set;
+
+/**
+ * Datetime sharding algorithm that adapt various shard method by define properties below.
+ *
+ * <p>properties defined here:
+ *
+ * <p>datetime.format: the datetime format used by applications, must can be transformed to {@link LocalDateTime},
+ * used by {@link LocalDateTime#parse(CharSequence, DateTimeFormatter)}.
+ *
+ * <p>table.suffix.format: suffix for sharded tables, used by {@link LocalDateTime#format(DateTimeFormatter)},
+ * examples:
+ * suffix=yyyyQQ means shard by {@link IsoFields#QUARTER_OF_YEAR};
+ * suffix=yyyyMMdd means shard by {@link ChronoField#DAY_OF_YEAR}.
+ *
+ * <p>detail explain for each char in datetime.format and table.suffix.format can refer {@link TemporalField}.
+ *
+ * <p>datetime.lower and datetime.upper: if app query with only half bound, lower and upper helps to build other half bound,
+ * datetime.lower must be specified and datetime.upper has a default value to {@link LocalDateTime#now}
+ * (default value of datetime.upper could only be used when query sql needn't get result that time larger than query time).
+ *
+ * <p>datetime.step.unit and datetime.step.amount used for calculate tables for range shard, datetime.step.unit is name of
+ * {@link ChronoUnit}, default unit is Days and amount is 1, amount + unit should not be larger than but close to your shard range.
+ *

Review comment:
       I like this detailed java doc.

##########
File path: sharding-core/sharding-core-common/src/test/java/org/apache/shardingsphere/core/strategy/algorithm/sharding/CustomDateTimeShardingAlgorithmTest.java
##########
@@ -0,0 +1,114 @@
+/*
+ * 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.shardingsphere.core.strategy.algorithm.sharding;
+
+import com.google.common.collect.Lists;
+import com.google.common.collect.Range;
+import org.apache.shardingsphere.api.config.sharding.strategy.StandardShardingStrategyConfiguration;
+import org.apache.shardingsphere.core.strategy.route.standard.StandardShardingStrategy;
+import org.apache.shardingsphere.core.strategy.route.value.ListRouteValue;
+import org.apache.shardingsphere.core.strategy.route.value.RangeRouteValue;
+import org.apache.shardingsphere.core.strategy.route.value.RouteValue;
+import org.apache.shardingsphere.underlying.common.config.properties.ConfigurationProperties;
+import org.junit.Before;
+import org.junit.Test;
+
+import java.time.LocalDateTime;
+import java.time.format.DateTimeFormatter;
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.List;
+import java.util.Properties;
+
+import static org.hamcrest.CoreMatchers.is;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertThat;
+import static org.junit.Assert.assertTrue;
+
+/**
+ * datetime sharding algorithm test.
+ */
+public class CustomDateTimeShardingAlgorithmTest {
+
+    private final List<String> availableTables = new ArrayList<>();
+
+    private StandardShardingStrategy shardingStrategy;
+
+    @Before
+    public void setup() {
+        CustomDateTimeShardingAlgorithm shardingAlgorithm = new CustomDateTimeShardingAlgorithm();
+        shardingAlgorithm.getProperties().setProperty("datetime.format", "yyyy-MM-dd HH:mm:ss");
+        shardingAlgorithm.getProperties().setProperty("table.suffix.format", "yyyyQQ");
+        shardingAlgorithm.getProperties().setProperty("datetime.lower", "2016-01-01 00:00:00.000");
+        shardingAlgorithm.getProperties().setProperty("datetime.upper", "2021-12-31 00:00:00.000");
+        shardingAlgorithm.getProperties().setProperty("datetime.step.unit", "Months");
+        shardingAlgorithm.getProperties().setProperty("datetime.step.amount", "3");
+        StandardShardingStrategyConfiguration shardingStrategyConfig = new StandardShardingStrategyConfiguration("create_time", shardingAlgorithm);
+        this.shardingStrategy = new StandardShardingStrategy(shardingStrategyConfig);
+

Review comment:
       A redundant blank line is recommended to remove.

##########
File path: sharding-core/sharding-core-common/src/test/java/org/apache/shardingsphere/core/strategy/algorithm/sharding/CustomDateTimeShardingAlgorithmTest.java
##########
@@ -0,0 +1,114 @@
+/*
+ * 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.shardingsphere.core.strategy.algorithm.sharding;
+
+import com.google.common.collect.Lists;
+import com.google.common.collect.Range;
+import org.apache.shardingsphere.api.config.sharding.strategy.StandardShardingStrategyConfiguration;
+import org.apache.shardingsphere.core.strategy.route.standard.StandardShardingStrategy;
+import org.apache.shardingsphere.core.strategy.route.value.ListRouteValue;
+import org.apache.shardingsphere.core.strategy.route.value.RangeRouteValue;
+import org.apache.shardingsphere.core.strategy.route.value.RouteValue;
+import org.apache.shardingsphere.underlying.common.config.properties.ConfigurationProperties;
+import org.junit.Before;
+import org.junit.Test;
+
+import java.time.LocalDateTime;
+import java.time.format.DateTimeFormatter;
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.List;
+import java.util.Properties;
+
+import static org.hamcrest.CoreMatchers.is;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertThat;
+import static org.junit.Assert.assertTrue;
+
+/**
+ * datetime sharding algorithm test.
+ */
+public class CustomDateTimeShardingAlgorithmTest {
+
+    private final List<String> availableTables = new ArrayList<>();
+
+    private StandardShardingStrategy shardingStrategy;
+
+    @Before
+    public void setup() {
+        CustomDateTimeShardingAlgorithm shardingAlgorithm = new CustomDateTimeShardingAlgorithm();
+        shardingAlgorithm.getProperties().setProperty("datetime.format", "yyyy-MM-dd HH:mm:ss");
+        shardingAlgorithm.getProperties().setProperty("table.suffix.format", "yyyyQQ");
+        shardingAlgorithm.getProperties().setProperty("datetime.lower", "2016-01-01 00:00:00.000");
+        shardingAlgorithm.getProperties().setProperty("datetime.upper", "2021-12-31 00:00:00.000");
+        shardingAlgorithm.getProperties().setProperty("datetime.step.unit", "Months");
+        shardingAlgorithm.getProperties().setProperty("datetime.step.amount", "3");
+        StandardShardingStrategyConfiguration shardingStrategyConfig = new StandardShardingStrategyConfiguration("create_time", shardingAlgorithm);
+        this.shardingStrategy = new StandardShardingStrategy(shardingStrategyConfig);
+
+        for (int i = 2016; i <= 2020; i++) {
+            for (int j = 1; j <= 4; j++) {
+                availableTables.add(String.format("t_order_%04d%02d", i, j));
+            }
+        }
+    }
+
+    @Test
+    public void assertPreciseDoSharding() {
+        List<RouteValue> shardingValues = Lists.newArrayList(new ListRouteValue<>("create_time", "t_order",
+                Lists.newArrayList("2020-01-01 00:00:01", "2020-01-01 00:00:02", "2020-04-15 10:59:08")));
+        Collection<String> actual = shardingStrategy.doSharding(availableTables, shardingValues, new ConfigurationProperties(new Properties()));
+        assertThat(actual.size(), is(2));
+        assertTrue(actual.contains("t_order_202001"));
+        assertTrue(actual.contains("t_order_202002"));
+    }
+
+    @Test
+    public void assertRangeDoSharding() {
+        Range<String> rangeValue = Range.closed("2019-10-15 10:59:08", "2020-04-08 10:59:08");
+        List<RouteValue> shardingValues = Lists.newArrayList(new RangeRouteValue<>("create_time", "t_order", rangeValue));
+        Collection<String> actual = shardingStrategy.doSharding(availableTables, shardingValues, new ConfigurationProperties(new Properties()));
+        assertThat(actual.size(), is(3));
+    }
+
+    @Test
+    public void assertLowerHalfRangeDoSharding() {
+        Range<String> rangeValue = Range.atLeast("2018-10-15 10:59:08");
+        List<RouteValue> shardingValues = Lists.newArrayList(new RangeRouteValue<>("create_time", "t_order", rangeValue));
+        Collection<String> actual = shardingStrategy.doSharding(availableTables, shardingValues, new ConfigurationProperties(new Properties()));
+        assertThat(actual.size(), is(9));
+    }
+
+    @Test
+    public void assertUpperHalfRangeDoSharding() {
+        Range<String> rangeValue = Range.atMost("2019-09-01 00:00:00");
+        List<RouteValue> shardingValues = Lists.newArrayList(new RangeRouteValue<>("create_time", "t_order", rangeValue));
+        Collection<String> actual = shardingStrategy.doSharding(availableTables, shardingValues, new ConfigurationProperties(new Properties()));
+        assertThat(actual.size(), is(15));
+    }
+
+    @Test
+    public void testFormat() {

Review comment:
       Considering the consistent function name, `assertFormat` is better.

##########
File path: sharding-core/sharding-core-common/src/test/java/org/apache/shardingsphere/core/strategy/algorithm/sharding/CustomDateTimeShardingAlgorithmTest.java
##########
@@ -0,0 +1,114 @@
+/*
+ * 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.shardingsphere.core.strategy.algorithm.sharding;
+
+import com.google.common.collect.Lists;
+import com.google.common.collect.Range;
+import org.apache.shardingsphere.api.config.sharding.strategy.StandardShardingStrategyConfiguration;
+import org.apache.shardingsphere.core.strategy.route.standard.StandardShardingStrategy;
+import org.apache.shardingsphere.core.strategy.route.value.ListRouteValue;
+import org.apache.shardingsphere.core.strategy.route.value.RangeRouteValue;
+import org.apache.shardingsphere.core.strategy.route.value.RouteValue;
+import org.apache.shardingsphere.underlying.common.config.properties.ConfigurationProperties;
+import org.junit.Before;
+import org.junit.Test;
+
+import java.time.LocalDateTime;
+import java.time.format.DateTimeFormatter;
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.List;
+import java.util.Properties;
+
+import static org.hamcrest.CoreMatchers.is;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertThat;
+import static org.junit.Assert.assertTrue;
+
+/**
+ * datetime sharding algorithm test.
+ */
+public class CustomDateTimeShardingAlgorithmTest {
+
+    private final List<String> availableTables = new ArrayList<>();
+
+    private StandardShardingStrategy shardingStrategy;
+
+    @Before
+    public void setup() {
+        CustomDateTimeShardingAlgorithm shardingAlgorithm = new CustomDateTimeShardingAlgorithm();

Review comment:
       This is a great practice t shard by quarter. Besides, could you provide another test case `like sharding by month or day` to present its flexibility to our users? 
   You know, a cool algorithm needs more examples to express its features.




----------------------------------------------------------------
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