You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@yunikorn.apache.org by ch...@apache.org on 2024/01/06 10:20:05 UTC

(yunikorn-core) branch master updated: [YUNIKORN-2260] Partition limit is NOT equivalent with the root limit (#763)

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

chia7712 pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/yunikorn-core.git


The following commit(s) were added to refs/heads/master by this push:
     new e677e8f3 [YUNIKORN-2260] Partition limit is NOT equivalent with the root limit (#763)
e677e8f3 is described below

commit e677e8f3dfa76dfcd03fb536ec024f0baf2d7f4b
Author: Kuan-Po Tseng <br...@gmail.com>
AuthorDate: Sat Jan 6 18:19:26 2024 +0800

    [YUNIKORN-2260] Partition limit is NOT equivalent with the root limit (#763)
    
    Closes: #763
    
    Signed-off-by: Chia-Ping Tsai <ch...@gmail.com>
---
 pkg/common/configs/config_test.go          | 19 +++++--
 pkg/common/configs/configvalidator.go      | 36 +++++++++++--
 pkg/common/configs/configvalidator_test.go | 85 ++++++++++++++++++++++++++++++
 3 files changed, 130 insertions(+), 10 deletions(-)

diff --git a/pkg/common/configs/config_test.go b/pkg/common/configs/config_test.go
index ec41f167..43c1c10e 100644
--- a/pkg/common/configs/config_test.go
+++ b/pkg/common/configs/config_test.go
@@ -22,6 +22,7 @@ import (
 	"crypto/sha256"
 	"encoding/json"
 	"fmt"
+	"reflect"
 	"strings"
 	"testing"
 
@@ -907,11 +908,11 @@ partitions:
 	conf, err := CreateConfig(data)
 	assert.NilError(t, err, "partition user parsing should not have failed")
 	// gone through validation: 1 top level queues
-	if len(conf.Partitions[0].Queues) != 1 {
-		t.Errorf("failed to load queue %v", conf)
+	if len(conf.Partitions[0].Queues) != 1 || len(conf.Partitions[0].Queues[0].Limits) != 3 {
+		t.Errorf("partition limits not correctly applied to root queue %v", conf)
 	}
 	if len(conf.Partitions[0].Limits) != 3 {
-		t.Errorf("partition users linked not correctly loaded  %v", conf)
+		t.Errorf("partition users linked not correctly loaded %v", conf)
 	}
 
 	// limit 1 check
@@ -948,6 +949,10 @@ partitions:
 	if limit.MaxApplications != 20 || limit.MaxResources != nil || len(limit.MaxResources) != 0 {
 		t.Errorf("loaded resource limits incorrectly (limit 3): %v", limit)
 	}
+
+	if !reflect.DeepEqual(conf.Partitions[0].Limits, conf.Partitions[0].Queues[0].Limits) {
+		t.Errorf("partition limits and root queue limits are not equivalent : %v", conf)
+	}
 }
 
 func TestQueueLimits(t *testing.T) {
@@ -1241,8 +1246,8 @@ partitions:
 	conf, err := CreateConfig(data)
 	assert.NilError(t, err, "config parsing should not have failed")
 	// gone through validation: 1 top level queues
-	if len(conf.Partitions[0].Queues) != 1 || len(conf.Partitions[0].Queues[0].Limits) != 0 {
-		t.Errorf("failed to load queues from config: %v", conf)
+	if len(conf.Partitions[0].Queues) != 1 || len(conf.Partitions[0].Queues[0].Limits) != 5 {
+		t.Errorf("partition limits not correctly applied to root queue: %v", conf)
 	}
 	if len(conf.Partitions[0].Limits) != 5 {
 		t.Errorf("failed to load partition limits from config: %v", conf)
@@ -1277,6 +1282,10 @@ partitions:
 	if len(limit.Groups) != 1 || limit.Groups[0] != "*" {
 		t.Errorf("failed to load wildcard group from config: %v", limit)
 	}
+
+	if !reflect.DeepEqual(conf.Partitions[0].Limits, conf.Partitions[0].Queues[0].Limits) {
+		t.Errorf("partition limits and root queue limits are not equivalent : %v", conf)
+	}
 }
 
 func TestLimitsFail(t *testing.T) {
diff --git a/pkg/common/configs/configvalidator.go b/pkg/common/configs/configvalidator.go
index 14e3815d..5e7a45b0 100644
--- a/pkg/common/configs/configvalidator.go
+++ b/pkg/common/configs/configvalidator.go
@@ -21,6 +21,7 @@ package configs
 import (
 	"fmt"
 	"math"
+	"reflect"
 	"regexp"
 	"strings"
 	"time"
@@ -595,6 +596,26 @@ func checkLimits(limits []Limit, obj string, queue *QueueConfig) error {
 	return nil
 }
 
+func checkLimitsStructure(partitionConfig *PartitionConfig) error {
+	partitionLimits := partitionConfig.Limits
+	rootQueue := &partitionConfig.Queues[0]
+
+	if len(partitionConfig.Queues) < 1 || strings.ToLower(rootQueue.Name) != RootQueue {
+		return fmt.Errorf("top queue name is %s not root", rootQueue.Name)
+	}
+
+	if len(partitionLimits) > 0 && len(rootQueue.Limits) > 0 && !reflect.DeepEqual(partitionLimits, rootQueue.Limits) {
+		return fmt.Errorf("partition limits and root queue limits are not equivalent")
+	}
+
+	// if root queue limits not defined, apply partition limits
+	if len(partitionLimits) > 0 && len(rootQueue.Limits) == 0 {
+		rootQueue.Limits = partitionLimits
+	}
+
+	return nil
+}
+
 // Check for global policy
 func checkNodeSortingPolicy(partition *PartitionConfig) error {
 	// get the policy
@@ -706,7 +727,7 @@ func checkQueuesStructure(partition *PartitionConfig) error {
 	if rootQueue.Resources.Guaranteed != nil || rootQueue.Resources.Max != nil {
 		return fmt.Errorf("root queue must not have resource limits set")
 	}
-	return checkQueues(&rootQueue, 1)
+	return nil
 }
 
 // Check the state dump file path, if configured, is a valid path that can be written to.
@@ -734,7 +755,8 @@ func Validate(newConfig *SchedulerConfig) error {
 
 	// check uniqueness
 	partitionMap := make(map[string]bool)
-	for i, partition := range newConfig.Partitions {
+	for i := range newConfig.Partitions {
+		partition := newConfig.Partitions[i]
 		if partition.Name == "" || strings.ToLower(partition.Name) == DefaultPartition {
 			partition.Name = DefaultPartition
 		}
@@ -747,15 +769,19 @@ func Validate(newConfig *SchedulerConfig) error {
 		if err != nil {
 			return err
 		}
-		_, err = checkQueueResource(partition.Queues[0], nil)
+		err = checkLimitsStructure(&partition)
 		if err != nil {
 			return err
 		}
-		err = checkPlacementRules(&partition)
+		err = checkQueues(&partition.Queues[0], 1)
 		if err != nil {
 			return err
 		}
-		err = checkLimits(partition.Limits, partition.Name, &partition.Queues[0])
+		_, err = checkQueueResource(partition.Queues[0], nil)
+		if err != nil {
+			return err
+		}
+		err = checkPlacementRules(&partition)
 		if err != nil {
 			return err
 		}
diff --git a/pkg/common/configs/configvalidator_test.go b/pkg/common/configs/configvalidator_test.go
index b8079b3d..8703ab59 100644
--- a/pkg/common/configs/configvalidator_test.go
+++ b/pkg/common/configs/configvalidator_test.go
@@ -1580,3 +1580,88 @@ func TestCheckLimits(t *testing.T) { //nolint:funlen
 		})
 	}
 }
+
+func TestCheckLimitsStructure(t *testing.T) {
+	userLimit := Limit{
+		Limit:           "user-limit",
+		Users:           []string{"test-user"},
+		MaxApplications: 100,
+		MaxResources:    map[string]string{"memory": "100"},
+	}
+	groupLimit := Limit{
+		Limit:           "group-limit",
+		Groups:          []string{"test-group"},
+		MaxApplications: 0,
+		MaxResources:    map[string]string{"memory": "100"},
+	}
+
+	// top queue is not root
+	partitionConfig := &PartitionConfig{
+		Name: DefaultPartition,
+		Queues: []QueueConfig{{
+			Name: "test",
+		}},
+	}
+	assert.Error(t, checkLimitsStructure(partitionConfig), "top queue name is test not root")
+
+	// partition limits and root queue limits are not equivalent
+	partitionConfig = &PartitionConfig{
+		Name: DefaultPartition,
+		Queues: []QueueConfig{{
+			Name:   RootQueue,
+			Limits: []Limit{userLimit},
+		}},
+		Limits: []Limit{groupLimit},
+	}
+	assert.Error(t, checkLimitsStructure(partitionConfig), "partition limits and root queue limits are not equivalent")
+
+	// partition limits and root queue limits are equivalent
+	partitionConfig = &PartitionConfig{
+		Name: DefaultPartition,
+		Queues: []QueueConfig{{
+			Name:   RootQueue,
+			Limits: []Limit{userLimit},
+		}},
+		Limits: []Limit{userLimit},
+	}
+	assert.NilError(t, checkLimitsStructure(partitionConfig))
+	assert.DeepEqual(t, partitionConfig.Queues[0].Limits, []Limit{userLimit})
+	assert.DeepEqual(t, partitionConfig.Limits, []Limit{userLimit})
+
+	// only partition limits exist
+	partitionConfig = &PartitionConfig{
+		Name: DefaultPartition,
+		Queues: []QueueConfig{{
+			Name: RootQueue,
+		}},
+		Limits: []Limit{groupLimit},
+	}
+	assert.NilError(t, checkLimitsStructure(partitionConfig))
+	assert.DeepEqual(t, partitionConfig.Queues[0].Limits, []Limit{groupLimit})
+	assert.DeepEqual(t, partitionConfig.Limits, []Limit{groupLimit})
+
+	// only root queue limits exist
+	partitionConfig = &PartitionConfig{
+		Name: DefaultPartition,
+		Queues: []QueueConfig{{
+			Name:   RootQueue,
+			Limits: []Limit{userLimit},
+		}},
+		Limits: []Limit{},
+	}
+	assert.NilError(t, checkLimitsStructure(partitionConfig))
+	assert.DeepEqual(t, partitionConfig.Queues[0].Limits, []Limit{userLimit})
+	assert.Equal(t, len(partitionConfig.Limits), 0)
+
+	// no limits exist
+	partitionConfig = &PartitionConfig{
+		Name: DefaultPartition,
+		Queues: []QueueConfig{{
+			Name: RootQueue,
+		}},
+		Limits: []Limit{},
+	}
+	assert.NilError(t, checkLimitsStructure(partitionConfig))
+	assert.Equal(t, len(partitionConfig.Queues[0].Limits), 0)
+	assert.Equal(t, len(partitionConfig.Limits), 0)
+}


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@yunikorn.apache.org
For additional commands, e-mail: issues-help@yunikorn.apache.org