You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@uniffle.apache.org by xi...@apache.org on 2023/02/02 07:55:03 UTC

[incubator-uniffle] branch master updated: [ISSUE-469][operator] feat: supports adding labels to rss pods. (#528)

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

xianjin pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-uniffle.git


The following commit(s) were added to refs/heads/master by this push:
     new 0d24c261 [ISSUE-469][operator] feat: supports adding labels to rss pods.  (#528)
0d24c261 is described below

commit 0d24c26196b6726d4729adfd537511f6b43f7e4f
Author: jasonawang <ja...@tencent.com>
AuthorDate: Thu Feb 2 15:54:58 2023 +0800

    [ISSUE-469][operator] feat: supports adding labels to rss pods.  (#528)
    
    ### What changes were proposed in this pull request?
    supports adding labels to rss pods
    
    ### Why are the changes needed?
    this fixes #469
    
    ### Does this PR introduce _any_ user-facing change?
    rss cluster admin cloud specify labels to rss coordinator or server
    
    ### How was this patch tested?
    Added ut.
---
 .../uniffle/v1alpha1/remoteshuffleservice_types.go |   4 +
 .../api/uniffle/v1alpha1/zz_generated.deepcopy.go  |   7 ++
 .../uniffle.apache.org_remoteshuffleservices.yaml  |  12 +++
 .../pkg/controller/controller/process_rss_test.go  |  19 +---
 .../operator/pkg/controller/controller/rss_test.go |  40 +-------
 .../controller/controller/shuffle_server_test.go   |  20 +---
 .../pkg/controller/sync/coordinator/coordinator.go |  21 +++--
 .../sync/coordinator/coordinator_test.go           |  90 ++++++++++++++++++
 .../controller/sync/shuffleserver/shuffleserver.go |  13 ++-
 .../sync/shuffleserver/shuffleserver_test.go       |  96 +++++++++++++++++++
 deploy/kubernetes/operator/pkg/utils/rss.go        | 102 +++++++++++++++++++++
 deploy/kubernetes/operator/pkg/utils/rss_test.go   |  71 ++++++++++++++
 12 files changed, 415 insertions(+), 80 deletions(-)

diff --git a/deploy/kubernetes/operator/api/uniffle/v1alpha1/remoteshuffleservice_types.go b/deploy/kubernetes/operator/api/uniffle/v1alpha1/remoteshuffleservice_types.go
index 2767e37b..e5025665 100644
--- a/deploy/kubernetes/operator/api/uniffle/v1alpha1/remoteshuffleservice_types.go
+++ b/deploy/kubernetes/operator/api/uniffle/v1alpha1/remoteshuffleservice_types.go
@@ -200,6 +200,10 @@ type RSSPodSpec struct {
 	// LogHostPath represents host path used to save logs of shuffle servers.
 	// +optional
 	LogHostPath string `json:"logHostPath,omitempty"`
+
+	// Labels represents labels to be added in coordinators or shuffle servers' pods.
+	// +optional
+	Labels map[string]string `json:"labels,omitempty"`
 }
 
 // MainContainer stores information of the main container of coordinators or shuffle servers,
diff --git a/deploy/kubernetes/operator/api/uniffle/v1alpha1/zz_generated.deepcopy.go b/deploy/kubernetes/operator/api/uniffle/v1alpha1/zz_generated.deepcopy.go
index 77a99dbf..4704e9bb 100644
--- a/deploy/kubernetes/operator/api/uniffle/v1alpha1/zz_generated.deepcopy.go
+++ b/deploy/kubernetes/operator/api/uniffle/v1alpha1/zz_generated.deepcopy.go
@@ -200,6 +200,13 @@ func (in *RSSPodSpec) DeepCopyInto(out *RSSPodSpec) {
 			(*out)[key] = val
 		}
 	}
+	if in.Labels != nil {
+		in, out := &in.Labels, &out.Labels
+		*out = make(map[string]string, len(*in))
+		for key, val := range *in {
+			(*out)[key] = val
+		}
+	}
 }
 
 // DeepCopy is an autogenerated deepcopy function, copying the receiver, creating a new RSSPodSpec.
diff --git a/deploy/kubernetes/operator/config/crd/bases/uniffle.apache.org_remoteshuffleservices.yaml b/deploy/kubernetes/operator/config/crd/bases/uniffle.apache.org_remoteshuffleservices.yaml
index 56b4bfcd..d45c2de1 100644
--- a/deploy/kubernetes/operator/config/crd/bases/uniffle.apache.org_remoteshuffleservices.yaml
+++ b/deploy/kubernetes/operator/config/crd/bases/uniffle.apache.org_remoteshuffleservices.yaml
@@ -240,6 +240,12 @@ spec:
                     description: InitContainerImage represents image of init container
                       used to change owner of host paths.
                     type: string
+                  labels:
+                    additionalProperties:
+                      type: string
+                    description: Labels represents labels to be added in coordinators
+                      or shuffle servers' pods.
+                    type: object
                   logHostPath:
                     description: LogHostPath represents host path used to save logs
                       of shuffle servers.
@@ -3429,6 +3435,12 @@ spec:
                     description: InitContainerImage represents image of init container
                       used to change owner of host paths.
                     type: string
+                  labels:
+                    additionalProperties:
+                      type: string
+                    description: Labels represents labels to be added in coordinators
+                      or shuffle servers' pods.
+                    type: object
                   logHostPath:
                     description: LogHostPath represents host path used to save logs
                       of shuffle servers.
diff --git a/deploy/kubernetes/operator/pkg/controller/controller/process_rss_test.go b/deploy/kubernetes/operator/pkg/controller/controller/process_rss_test.go
index a5ac8748..f52ca3a9 100644
--- a/deploy/kubernetes/operator/pkg/controller/controller/process_rss_test.go
+++ b/deploy/kubernetes/operator/pkg/controller/controller/process_rss_test.go
@@ -31,26 +31,9 @@ import (
 	"github.com/apache/incubator-uniffle/deploy/kubernetes/operator/pkg/utils"
 )
 
-// buildEmptyPhaseRssObj builds a rss object with empty phase for testing.
-func buildEmptyPhaseRssObj() *unifflev1alpha1.RemoteShuffleService {
-	return &unifflev1alpha1.RemoteShuffleService{
-		ObjectMeta: metav1.ObjectMeta{
-			Name:            testRssName,
-			Namespace:       testNamespace,
-			ResourceVersion: "test",
-		},
-		Spec: unifflev1alpha1.RemoteShuffleServiceSpec{
-			Coordinator: &unifflev1alpha1.CoordinatorConfig{
-				ExcludeNodesFilePath: "/exclude_nodes",
-			},
-		},
-		Status: unifflev1alpha1.RemoteShuffleServiceStatus{},
-	}
-}
-
 // TestProcessEmptyPhaseRss tests rss objects' process of rss-controller
 func TestProcessEmptyPhaseRss(t *testing.T) {
-	rss := buildEmptyPhaseRssObj()
+	rss := utils.BuildRSSWithDefaultValue()
 
 	rssClient := fake.NewSimpleClientset(rss)
 	kubeClient := kubefake.NewSimpleClientset()
diff --git a/deploy/kubernetes/operator/pkg/controller/controller/rss_test.go b/deploy/kubernetes/operator/pkg/controller/controller/rss_test.go
index ac905bb2..c2c3c1a4 100644
--- a/deploy/kubernetes/operator/pkg/controller/controller/rss_test.go
+++ b/deploy/kubernetes/operator/pkg/controller/controller/rss_test.go
@@ -283,40 +283,10 @@ func initTestRss() (*corev1.ConfigMap, *unifflev1alpha1.RemoteShuffleService) {
 			constants.Log4jPropertiesKey:     "",
 		},
 	}
-	rss := &unifflev1alpha1.RemoteShuffleService{
-		ObjectMeta: metav1.ObjectMeta{
-			Name:      testRssName,
-			Namespace: testNamespace,
-		},
-		Spec: unifflev1alpha1.RemoteShuffleServiceSpec{
-			ConfigMapName: testRssName,
-			Coordinator: &unifflev1alpha1.CoordinatorConfig{
-				HTTPNodePort: []int32{30001, 30011},
-				RPCNodePort:  []int32{30002, 30012},
-				CommonConfig: &unifflev1alpha1.CommonConfig{
-					ConfigDir: "/app/config",
-					RSSPodSpec: &unifflev1alpha1.RSSPodSpec{
-						MainContainer: &unifflev1alpha1.MainContainer{
-							Image: testCoordinatorImage1,
-						},
-					},
-				},
-			},
-			ShuffleServer: &unifflev1alpha1.ShuffleServerConfig{
-				CommonConfig: &unifflev1alpha1.CommonConfig{
-					ConfigDir: "/app/config",
-					RSSPodSpec: &unifflev1alpha1.RSSPodSpec{
-						MainContainer: &unifflev1alpha1.MainContainer{
-							Image: "rss-shuffleserver:latest",
-						},
-					},
-					XmxSize: "10G",
-				},
-				UpgradeStrategy: &unifflev1alpha1.ShuffleServerUpgradeStrategy{
-					Type: unifflev1alpha1.FullUpgrade,
-				},
-			},
-		},
-	}
+	rss := utils.BuildRSSWithDefaultValue()
+	rss.Spec.ConfigMapName = cm.Name
+	rss.Name = testRssName
+	rss.Namespace = testNamespace
+	rss.Spec.Coordinator.Image = testCoordinatorImage1
 	return cm, rss
 }
diff --git a/deploy/kubernetes/operator/pkg/controller/controller/shuffle_server_test.go b/deploy/kubernetes/operator/pkg/controller/controller/shuffle_server_test.go
index 669bf318..749ae1c2 100644
--- a/deploy/kubernetes/operator/pkg/controller/controller/shuffle_server_test.go
+++ b/deploy/kubernetes/operator/pkg/controller/controller/shuffle_server_test.go
@@ -35,22 +35,12 @@ import (
 )
 
 func buildUpgradingRssObjWithTargetKeys(shuffleServerKey string) *unifflev1alpha1.RemoteShuffleService {
-	return &unifflev1alpha1.RemoteShuffleService{
-		ObjectMeta: metav1.ObjectMeta{
-			Name:            testRssName,
-			Namespace:       testNamespace,
-			ResourceVersion: "test",
-		},
-		Spec: unifflev1alpha1.RemoteShuffleServiceSpec{
-			Coordinator: &unifflev1alpha1.CoordinatorConfig{
-				ExcludeNodesFilePath: "/exclude_nodes",
-			},
-		},
-		Status: unifflev1alpha1.RemoteShuffleServiceStatus{
-			Phase:      unifflev1alpha1.RSSUpgrading,
-			TargetKeys: []string{shuffleServerKey},
-		},
+	rss := utils.BuildRSSWithDefaultValue()
+	rss.Status = unifflev1alpha1.RemoteShuffleServiceStatus{
+		Phase:      unifflev1alpha1.RSSUpgrading,
+		TargetKeys: []string{shuffleServerKey},
 	}
+	return rss
 }
 
 func buildTestShuffleServerPod() *corev1.Pod {
diff --git a/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator.go b/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator.go
index 5ce3a562..45140a47 100644
--- a/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator.go
+++ b/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator.go
@@ -160,31 +160,34 @@ func GenerateDeploy(rss *unifflev1alpha1.RemoteShuffleService, index int) *appsv
 		podSpec.DNSPolicy = corev1.DNSClusterFirstWithHostNet
 	}
 
+	defaultLabels := map[string]string{
+		"app": name,
+	}
 	deploy := &appsv1.Deployment{
 		ObjectMeta: metav1.ObjectMeta{
 			Name:      name,
 			Namespace: rss.Namespace,
-			Labels: map[string]string{
-				"app": name,
-			},
+			Labels:    defaultLabels,
 		},
 		Spec: appsv1.DeploymentSpec{
 			Selector: &metav1.LabelSelector{
-				MatchLabels: map[string]string{
-					"app": name,
-				},
+				MatchLabels: defaultLabels,
 			},
 			Replicas: rss.Spec.Coordinator.Replicas,
 			Template: corev1.PodTemplateSpec{
 				ObjectMeta: metav1.ObjectMeta{
-					Labels: map[string]string{
-						"app": name,
-					},
+					Labels: make(map[string]string),
 				},
 				Spec: podSpec,
 			},
 		},
 	}
+	for k, v := range rss.Spec.Coordinator.Labels {
+		deploy.Spec.Template.Labels[k] = v
+	}
+	for k, v := range defaultLabels {
+		deploy.Spec.Template.Labels[k] = v
+	}
 
 	// add init containers, the main container and other containers.
 	deploy.Spec.Template.Spec.InitContainers = util.GenerateInitContainers(rss.Spec.Coordinator.RSSPodSpec)
diff --git a/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator_test.go b/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator_test.go
new file mode 100644
index 00000000..89ee7220
--- /dev/null
+++ b/deploy/kubernetes/operator/pkg/controller/sync/coordinator/coordinator_test.go
@@ -0,0 +1,90 @@
+/*
+ * 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 coordinator
+
+import (
+	"fmt"
+	"testing"
+
+	appsv1 "k8s.io/api/apps/v1"
+
+	unifflev1alpha1 "github.com/apache/incubator-uniffle/deploy/kubernetes/operator/api/uniffle/v1alpha1"
+	"github.com/apache/incubator-uniffle/deploy/kubernetes/operator/pkg/utils"
+)
+
+// IsValidDeploy checks generated deployment, returns whether it is valid and error message.
+type IsValidDeploy func(*appsv1.Deployment) (bool, error)
+
+var commonLabels = map[string]string{
+	"key1": "value1",
+	"key2": "value2",
+	"key3": "value3",
+}
+
+func buildRssWithLabels() *unifflev1alpha1.RemoteShuffleService {
+	rss := utils.BuildRSSWithDefaultValue()
+	rss.Spec.Coordinator.Labels = commonLabels
+	return rss
+}
+
+func TestGenerateDeploy(t *testing.T) {
+	for _, tt := range []struct {
+		name string
+		rss  *unifflev1alpha1.RemoteShuffleService
+		IsValidDeploy
+	}{
+		{
+			name: "add custom labels",
+			rss:  buildRssWithLabels(),
+			IsValidDeploy: func(deploy *appsv1.Deployment) (bool, error) {
+				var valid = true
+				var err error
+
+				expectedLabels := map[string]string{
+					"app": "rss-coordinator-rss-0",
+				}
+				for k := range commonLabels {
+					expectedLabels[k] = commonLabels[k]
+				}
+
+				currentLabels := deploy.Spec.Template.Labels
+				if len(expectedLabels) != len(currentLabels) {
+					valid = false
+				} else {
+					for k := range currentLabels {
+						if expectedLabels[k] != currentLabels[k] {
+							valid = false
+							break
+						}
+					}
+				}
+				if !valid {
+					err = fmt.Errorf("unexpected labels: %+v, expected: %+v", currentLabels, expectedLabels)
+				}
+				return valid, err
+			},
+		},
+	} {
+		t.Run(tt.name, func(tc *testing.T) {
+			deploy := GenerateDeploy(tt.rss, 0)
+			if valid, err := tt.IsValidDeploy(deploy); !valid {
+				tc.Error(err)
+			}
+		})
+	}
+}
diff --git a/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver.go b/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver.go
index cfab93d9..c461049d 100644
--- a/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver.go
+++ b/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver.go
@@ -175,15 +175,16 @@ func GenerateSts(rss *unifflev1alpha1.RemoteShuffleService) *appsv1.StatefulSet
 		podSpec.DNSPolicy = corev1.DNSClusterFirstWithHostNet
 	}
 
+	defaultLabels := utils.GenerateShuffleServerLabels(rss)
 	sts := &appsv1.StatefulSet{
 		ObjectMeta: metav1.ObjectMeta{
 			Name:      name,
 			Namespace: rss.Namespace,
-			Labels:    utils.GenerateShuffleServerLabels(rss),
+			Labels:    defaultLabels,
 		},
 		Spec: appsv1.StatefulSetSpec{
 			Selector: &metav1.LabelSelector{
-				MatchLabels: utils.GenerateShuffleServerLabels(rss),
+				MatchLabels: defaultLabels,
 			},
 			UpdateStrategy: appsv1.StatefulSetUpdateStrategy{
 				Type: appsv1.RollingUpdateStatefulSetStrategyType,
@@ -195,7 +196,7 @@ func GenerateSts(rss *unifflev1alpha1.RemoteShuffleService) *appsv1.StatefulSet
 			Replicas:    replicas,
 			Template: corev1.PodTemplateSpec{
 				ObjectMeta: metav1.ObjectMeta{
-					Labels: utils.GenerateShuffleServerLabels(rss),
+					Labels: make(map[string]string),
 					Annotations: map[string]string{
 						constants.AnnotationRssName: rss.Name,
 						constants.AnnotationRssUID:  string(rss.UID),
@@ -209,6 +210,12 @@ func GenerateSts(rss *unifflev1alpha1.RemoteShuffleService) *appsv1.StatefulSet
 			},
 		},
 	}
+	for k, v := range rss.Spec.ShuffleServer.Labels {
+		sts.Spec.Template.Labels[k] = v
+	}
+	for k, v := range defaultLabels {
+		sts.Spec.Template.Labels[k] = v
+	}
 
 	// add init containers, the main container and other containers.
 	sts.Spec.Template.Spec.InitContainers = util.GenerateInitContainers(rss.Spec.ShuffleServer.RSSPodSpec)
diff --git a/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver_test.go b/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver_test.go
new file mode 100644
index 00000000..ede041de
--- /dev/null
+++ b/deploy/kubernetes/operator/pkg/controller/sync/shuffleserver/shuffleserver_test.go
@@ -0,0 +1,96 @@
+/*
+ * 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 shuffleserver
+
+import (
+	"fmt"
+	"testing"
+
+	appsv1 "k8s.io/api/apps/v1"
+
+	unifflev1alpha1 "github.com/apache/incubator-uniffle/deploy/kubernetes/operator/api/uniffle/v1alpha1"
+	"github.com/apache/incubator-uniffle/deploy/kubernetes/operator/pkg/utils"
+)
+
+// IsValidSts checks generated statefulSet, returns whether it is valid and error message.
+type IsValidSts func(*appsv1.StatefulSet) (bool, error)
+
+var commonLabels = map[string]string{
+	"key1": "value1",
+	"key2": "value2",
+	"key3": "value3",
+}
+
+func buildRssWithLabels() *unifflev1alpha1.RemoteShuffleService {
+	rss := utils.BuildRSSWithDefaultValue()
+	rss.Spec.ShuffleServer.Labels = map[string]string{
+		"uniffle.apache.org/shuffle-server": "change-test",
+	}
+	for k := range commonLabels {
+		rss.Spec.ShuffleServer.Labels[k] = commonLabels[k]
+	}
+	return rss
+}
+
+func TestGenerateSts(t *testing.T) {
+	for _, tt := range []struct {
+		name string
+		rss  *unifflev1alpha1.RemoteShuffleService
+		IsValidSts
+	}{
+		{
+			name: "add custom labels",
+			rss:  buildRssWithLabels(),
+			IsValidSts: func(sts *appsv1.StatefulSet) (bool, error) {
+				var valid = true
+				var err error
+
+				expectedLabels := map[string]string{
+					"app":                               "rss-shuffle-server-rss",
+					"uniffle.apache.org/shuffle-server": "true",
+				}
+				for k := range commonLabels {
+					expectedLabels[k] = commonLabels[k]
+				}
+
+				currentLabels := sts.Spec.Template.Labels
+				if len(expectedLabels) != len(currentLabels) {
+					valid = false
+				} else {
+					for k := range currentLabels {
+						if expectedLabels[k] != currentLabels[k] {
+							valid = false
+							break
+						}
+					}
+				}
+				if !valid {
+					err = fmt.Errorf("unexpected labels: %+v, expected: %+v", currentLabels, expectedLabels)
+				}
+				return valid, err
+			},
+		},
+	} {
+		t.Run(tt.name, func(tc *testing.T) {
+			sts := GenerateSts(tt.rss)
+			if valid, err := tt.IsValidSts(sts); !valid {
+				tc.Error(err)
+			}
+		})
+	}
+}
diff --git a/deploy/kubernetes/operator/pkg/utils/rss.go b/deploy/kubernetes/operator/pkg/utils/rss.go
new file mode 100644
index 00000000..96346470
--- /dev/null
+++ b/deploy/kubernetes/operator/pkg/utils/rss.go
@@ -0,0 +1,102 @@
+/*
+ * 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 utils
+
+import (
+	corev1 "k8s.io/api/core/v1"
+	metav1 "k8s.io/apimachinery/pkg/apis/meta/v1"
+	"k8s.io/utils/pointer"
+
+	unifflev1alpha1 "github.com/apache/incubator-uniffle/deploy/kubernetes/operator/api/uniffle/v1alpha1"
+)
+
+const (
+	defaultRssName              = "rss"
+	defaultConfigMapName        = "rss-config"
+	defaultCoordinatorSync      = true
+	defaultCoordinatorCount     = 2
+	defaultExcludeNodesFilePath = "/config/exclude_nodes"
+	defaultShuffleServerSync    = false
+	defaultRPCPort              = 19997
+	defaultHTTPPort             = 19996
+	defaultReplicas             = 1
+	defaultXmxSize              = "800M"
+	defaultConfigDir            = "/data/rssadmin/rss/conf"
+	defaultHostNetwork          = true
+	defaultMainImage            = "rss-server:latest"
+)
+
+var (
+	defaultCoordinatorRPCNodePorts  = []int32{30001, 30011}
+	defaultCoordinatorHTTPNodePorts = []int32{30002, 30012}
+)
+
+// BuildRSSWithDefaultValue builds a rss object with required or default values for testing.
+func BuildRSSWithDefaultValue() *unifflev1alpha1.RemoteShuffleService {
+	return &unifflev1alpha1.RemoteShuffleService{
+		ObjectMeta: metav1.ObjectMeta{
+			Name:      defaultRssName,
+			Namespace: corev1.NamespaceDefault,
+		},
+		Spec: unifflev1alpha1.RemoteShuffleServiceSpec{
+			Coordinator:   buildCoordinatorConfigWithDefaultValue(),
+			ShuffleServer: buildShuffleServerConfigWithDefaultValue(),
+			ConfigMapName: defaultConfigMapName,
+		},
+	}
+}
+
+func buildCoordinatorConfigWithDefaultValue() *unifflev1alpha1.CoordinatorConfig {
+	return &unifflev1alpha1.CoordinatorConfig{
+		CommonConfig:         buildCommonConfig(),
+		Sync:                 pointer.Bool(defaultCoordinatorSync),
+		Count:                pointer.Int32(defaultCoordinatorCount),
+		Replicas:             pointer.Int32(defaultReplicas),
+		RPCPort:              pointer.Int32(defaultRPCPort),
+		HTTPPort:             pointer.Int32(defaultHTTPPort),
+		ExcludeNodesFilePath: defaultExcludeNodesFilePath,
+		RPCNodePort:          defaultCoordinatorRPCNodePorts,
+		HTTPNodePort:         defaultCoordinatorHTTPNodePorts,
+	}
+}
+
+func buildShuffleServerConfigWithDefaultValue() *unifflev1alpha1.ShuffleServerConfig {
+	return &unifflev1alpha1.ShuffleServerConfig{
+		CommonConfig: buildCommonConfig(),
+		Sync:         pointer.Bool(defaultShuffleServerSync),
+		Replicas:     pointer.Int32(defaultReplicas),
+		RPCPort:      pointer.Int32(defaultRPCPort),
+		HTTPPort:     pointer.Int32(defaultHTTPPort),
+		UpgradeStrategy: &unifflev1alpha1.ShuffleServerUpgradeStrategy{
+			Type: unifflev1alpha1.FullUpgrade,
+		},
+	}
+}
+
+func buildCommonConfig() *unifflev1alpha1.CommonConfig {
+	return &unifflev1alpha1.CommonConfig{
+		RSSPodSpec: &unifflev1alpha1.RSSPodSpec{
+			MainContainer: &unifflev1alpha1.MainContainer{
+				Image: defaultMainImage,
+			},
+			HostNetwork: pointer.Bool(defaultHostNetwork),
+		},
+		XmxSize:   defaultXmxSize,
+		ConfigDir: defaultConfigDir,
+	}
+}
diff --git a/deploy/kubernetes/operator/pkg/utils/rss_test.go b/deploy/kubernetes/operator/pkg/utils/rss_test.go
new file mode 100644
index 00000000..be19074c
--- /dev/null
+++ b/deploy/kubernetes/operator/pkg/utils/rss_test.go
@@ -0,0 +1,71 @@
+/*
+ * 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 utils
+
+import (
+	"testing"
+
+	"github.com/stretchr/testify/assert"
+	"k8s.io/utils/pointer"
+
+	unifflev1alpha1 "github.com/apache/incubator-uniffle/deploy/kubernetes/operator/api/uniffle/v1alpha1"
+)
+
+func TestBuildRSSWithDefaultValue(t *testing.T) {
+	rss := BuildRSSWithDefaultValue()
+	assertion := assert.New(t)
+	assertion.NotEmpty(rss.Name, "need to specify name of rss")
+	assertion.NotEmpty(rss.Namespace, "need to specify namespace of rss")
+	assertion.NotEmpty(rss.Spec.Coordinator, "need to configure coordinator")
+	assertion.NotEmpty(rss.Spec.ShuffleServer, "need to configure shuffle server")
+	assertion.NotEmpty(rss.Spec.ConfigMapName, "need to configure configMap name of rss")
+
+	// check config of coordinator.
+	assertion.NotEmpty(rss.Spec.Coordinator.CommonConfig, "need to configure commonConfig of coordinator")
+	checkCommonConfig(assertion, rss.Spec.Coordinator.CommonConfig)
+	assertion.Equal(rss.Spec.Coordinator.Sync, pointer.Bool(defaultCoordinatorSync))
+	assertion.Equal(rss.Spec.Coordinator.Count, pointer.Int32(defaultCoordinatorCount))
+	assertion.Equal(rss.Spec.Coordinator.Replicas, pointer.Int32(defaultReplicas))
+	assertion.Equal(rss.Spec.Coordinator.RPCPort, pointer.Int32(defaultRPCPort))
+	assertion.Equal(rss.Spec.Coordinator.HTTPPort, pointer.Int32(defaultHTTPPort))
+	assertion.Equal(rss.Spec.Coordinator.ExcludeNodesFilePath, defaultExcludeNodesFilePath)
+	assertion.Equal(rss.Spec.Coordinator.RPCNodePort, defaultCoordinatorRPCNodePorts)
+	assertion.Equal(rss.Spec.Coordinator.HTTPNodePort, defaultCoordinatorHTTPNodePorts)
+
+	// check config of shuffle server.
+	assertion.NotEmpty(rss.Spec.ShuffleServer.CommonConfig, "need to configure commonConfig of shuffle server")
+	checkCommonConfig(assertion, rss.Spec.ShuffleServer.CommonConfig)
+	assertion.Equal(rss.Spec.ShuffleServer.Sync, pointer.Bool(defaultShuffleServerSync))
+	assertion.Equal(rss.Spec.ShuffleServer.Replicas, pointer.Int32(defaultReplicas))
+	assertion.Equal(rss.Spec.ShuffleServer.RPCPort, pointer.Int32(defaultRPCPort))
+	assertion.Equal(rss.Spec.ShuffleServer.HTTPPort, pointer.Int32(defaultHTTPPort))
+	assertion.Equal(rss.Spec.ShuffleServer.UpgradeStrategy, &unifflev1alpha1.ShuffleServerUpgradeStrategy{
+		Type: unifflev1alpha1.FullUpgrade,
+	})
+}
+
+func checkCommonConfig(assertion *assert.Assertions, commonConfig *unifflev1alpha1.CommonConfig) {
+	assertion.Equal(commonConfig.RSSPodSpec, &unifflev1alpha1.RSSPodSpec{
+		MainContainer: &unifflev1alpha1.MainContainer{
+			Image: defaultMainImage,
+		},
+		HostNetwork: pointer.Bool(defaultHostNetwork),
+	})
+	assertion.Equal(commonConfig.XmxSize, defaultXmxSize)
+	assertion.Equal(commonConfig.ConfigDir, defaultConfigDir)
+}