You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@linkis.apache.org by ca...@apache.org on 2023/03/07 11:28:12 UTC

[linkis] branch dev-1.3.2 updated: [Feature][LinkisManager] Add node select Hotspot exclusion rules (#4320)

This is an automated email from the ASF dual-hosted git repository.

casion pushed a commit to branch dev-1.3.2
in repository https://gitbox.apache.org/repos/asf/linkis.git


The following commit(s) were added to refs/heads/dev-1.3.2 by this push:
     new ed878cdab [Feature][LinkisManager] Add node select Hotspot exclusion rules  (#4320)
ed878cdab is described below

commit ed878cdabb428d348479e312d5c57d41a95d93e9
Author: peacewong <wp...@gmail.com>
AuthorDate: Tue Mar 7 19:28:05 2023 +0800

    [Feature][LinkisManager] Add node select Hotspot exclusion rules  (#4320)
    
    * Add Hotspot exclusion rules, random ordering of preceding elements
---
 .../impl/ParserVarLabelInterceptor.scala           |  3 +-
 .../linkis/manager/am/conf/AMConfiguration.scala   |  3 +
 .../am/selector/rule/HotspotExclusionRule.scala    | 70 ++++++++++++++++++++
 .../selector/rule/HotspotExclusionRuleTest.scala   | 76 ++++++++++++++++++++++
 4 files changed, 151 insertions(+), 1 deletion(-)

diff --git a/linkis-computation-governance/linkis-entrance/src/main/scala/org/apache/linkis/entrance/interceptor/impl/ParserVarLabelInterceptor.scala b/linkis-computation-governance/linkis-entrance/src/main/scala/org/apache/linkis/entrance/interceptor/impl/ParserVarLabelInterceptor.scala
index 020a0659f..0d3f074dc 100644
--- a/linkis-computation-governance/linkis-entrance/src/main/scala/org/apache/linkis/entrance/interceptor/impl/ParserVarLabelInterceptor.scala
+++ b/linkis-computation-governance/linkis-entrance/src/main/scala/org/apache/linkis/entrance/interceptor/impl/ParserVarLabelInterceptor.scala
@@ -22,6 +22,7 @@ import org.apache.linkis.governance.common.entity.job.JobRequest
 import org.apache.linkis.manager.label.builder.factory.LabelBuilderFactoryContext
 import org.apache.linkis.manager.label.constant.LabelKeyConstant
 import org.apache.linkis.manager.label.entity.TenantLabel
+import org.apache.linkis.protocol.utils.TaskUtils
 
 import java.lang
 
@@ -30,7 +31,7 @@ class ParserVarLabelInterceptor extends EntranceInterceptor {
   override def apply(jobRequest: JobRequest, logAppender: lang.StringBuilder): JobRequest = {
     jobRequest match {
       case requestPersistTask: JobRequest =>
-        val variableMap = requestPersistTask.getParams
+        val variableMap = TaskUtils.getVariableMap(requestPersistTask.getParams)
         val labels = requestPersistTask.getLabels
         if (variableMap.containsKey(LabelKeyConstant.TENANT_KEY)) {
           val tenantLabel = LabelBuilderFactoryContext.getLabelBuilderFactory
diff --git a/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/conf/AMConfiguration.scala b/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/conf/AMConfiguration.scala
index 5d010efda..c730edf0f 100644
--- a/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/conf/AMConfiguration.scala
+++ b/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/conf/AMConfiguration.scala
@@ -87,6 +87,9 @@ object AMConfiguration {
   val EC_MAINTAIN_WORK_END_TIME =
     CommonVars("wds.linkis.ec.maintain.time.work.end.time", 19).getValue
 
+  val NODE_SELECT_HOTSPOT_EXCLUSION_RULE =
+    CommonVars("linkis.node.select.hotspot.exclusion.rule.enable", true).getValue
+
   private def getDefaultMultiEngineUser(): String = {
     val jvmUser = Utils.getJvmUser
     s""" {jdbc:"$jvmUser", es: "$jvmUser", presto:"$jvmUser", appconn:"$jvmUser", openlookeng:"$jvmUser", trino:"$jvmUser", io_file:"root"}"""
diff --git a/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/selector/rule/HotspotExclusionRule.scala b/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/selector/rule/HotspotExclusionRule.scala
new file mode 100644
index 000000000..74d938316
--- /dev/null
+++ b/linkis-computation-governance/linkis-manager/linkis-application-manager/src/main/scala/org/apache/linkis/manager/am/selector/rule/HotspotExclusionRule.scala
@@ -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.linkis.manager.am.selector.rule
+
+import org.apache.linkis.common.utils.Logging
+import org.apache.linkis.manager.am.conf.AMConfiguration
+import org.apache.linkis.manager.common.entity.node.Node
+
+import org.springframework.core.annotation.Order
+import org.springframework.stereotype.Component
+
+import scala.util.Random
+
+/**
+ * Ten hotspot exclusion rules, random ordering of the previous elements
+ */
+@Component
+@Order(Int.MaxValue)
+class HotspotExclusionRule extends NodeSelectRule with Logging {
+
+  override def ruleFiltering(nodes: Array[Node]): Array[Node] = {
+    if (AMConfiguration.NODE_SELECT_HOTSPOT_EXCLUSION_RULE) {
+      randomShuffle(nodes)
+    } else {
+      nodes
+    }
+  }
+
+  /**
+   * Random sorting if there are more than 10 sorting rules, the first 5 will be randomly sorted if
+   * there are no more than 10, the first half will be sorted
+   * @param nodes
+   * @return
+   */
+  private def randomShuffle(nodes: Array[Node]): Array[Node] = {
+    if (null == nodes || nodes.length < 3) {
+      return nodes
+    }
+    val shuffleSize = if (nodes.length <= 10) {
+      nodes.length / 2
+    } else {
+      5
+    }
+    val random = new Random()
+
+    for (i <- 0 until shuffleSize) {
+      val nextIndex = random.nextInt(shuffleSize)
+      val tmpValue = nodes(i)
+      nodes(i) = nodes(nextIndex)
+      nodes(nextIndex) = tmpValue
+    }
+    nodes
+  }
+
+}
diff --git a/linkis-computation-governance/linkis-manager/linkis-application-manager/src/test/scala/org/apache/linkis/manager/am/selector/rule/HotspotExclusionRuleTest.scala b/linkis-computation-governance/linkis-manager/linkis-application-manager/src/test/scala/org/apache/linkis/manager/am/selector/rule/HotspotExclusionRuleTest.scala
new file mode 100644
index 000000000..9916daea8
--- /dev/null
+++ b/linkis-computation-governance/linkis-manager/linkis-application-manager/src/test/scala/org/apache/linkis/manager/am/selector/rule/HotspotExclusionRuleTest.scala
@@ -0,0 +1,76 @@
+/*
+ * 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.linkis.manager.am.selector.rule
+
+import org.apache.linkis.common.ServiceInstance
+import org.apache.linkis.manager.common.entity.node.AMEMNode
+
+import scala.collection.mutable.ArrayBuffer
+
+import org.junit.jupiter.api.Test
+
+class HotspotExclusionRuleTest {
+
+  @Test def testRuleFiltering(): Unit = {
+    val hotspotExclusionRule = new HotspotExclusionRule()
+    val bufferNodes = new ArrayBuffer[AMEMNode]()
+    for (i <- 0 until 3) {
+      val amEmNode = new AMEMNode();
+      amEmNode.setServiceInstance(ServiceInstance("ecm", s"ecm:$i"))
+      bufferNodes.append(amEmNode)
+    }
+    val res = hotspotExclusionRule.ruleFiltering(bufferNodes.toArray)
+    for (i <- 0 until 3) {
+      assert(res(i).getServiceInstance.equals(bufferNodes(i).getServiceInstance))
+    }
+  }
+
+  @Test def testRandomFiltering(): Unit = {
+    val hotspotExclusionRule = new HotspotExclusionRule()
+    val bufferNodes = new ArrayBuffer[AMEMNode]()
+    for (i <- 0 until 9) {
+      val amEmNode = new AMEMNode();
+      amEmNode.setServiceInstance(ServiceInstance("ecm", s"ecm:$i"))
+      bufferNodes.append(amEmNode)
+    }
+    val res = hotspotExclusionRule.ruleFiltering(bufferNodes.toArray)
+    for (i <- 4 until 9) {
+      assert(res(i).getServiceInstance.equals(bufferNodes(i).getServiceInstance))
+    }
+  }
+
+  @Test def testRandomFilteringOverTen(): Unit = {
+    val hotspotExclusionRule = new HotspotExclusionRule()
+    val bufferNodes = new ArrayBuffer[AMEMNode]()
+    for (i <- 0 until 15) {
+      val amEmNode = new AMEMNode();
+      amEmNode.setServiceInstance(ServiceInstance("ecm", s"ecm:$i"))
+      bufferNodes.append(amEmNode)
+    }
+    val res = hotspotExclusionRule.ruleFiltering(bufferNodes.toArray)
+    for (i <- 0 until 4) {
+      // scalastyle:off println
+      println(res(i).getServiceInstance)
+      // scalastyle:on println
+    }
+    for (i <- 5 until 15) {
+      assert(res(i).getServiceInstance.equals(bufferNodes(i).getServiceInstance))
+    }
+  }
+
+}


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