diff --git a/controllers/compaction_lease_controller.go b/controllers/compaction_lease_controller.go index 66aa48039..686e7c763 100644 --- a/controllers/compaction_lease_controller.go +++ b/controllers/compaction_lease_controller.go @@ -40,7 +40,6 @@ import ( druidv1alpha1 "github.com/gardener/etcd-druid/api/v1alpha1" controllersconfig "github.com/gardener/etcd-druid/controllers/config" "github.com/gardener/etcd-druid/pkg/common" - componentlease "github.com/gardener/etcd-druid/pkg/component/etcd/lease" druidpredicates "github.com/gardener/etcd-druid/pkg/predicate" "github.com/gardener/etcd-druid/pkg/utils" "github.com/gardener/gardener/pkg/utils/imagevector" @@ -119,7 +118,7 @@ func (lc *CompactionLeaseController) Reconcile(ctx context.Context, req ctrl.Req // Get full and delta snapshot lease to check the HolderIdentity value to take decision on compaction job fullLease := &coordinationv1.Lease{} - if err := lc.Get(ctx, kutil.Key(etcd.Namespace, componentlease.GetFullSnapshotLeaseName(etcd)), fullLease); err != nil { + if err := lc.Get(ctx, kutil.Key(etcd.Namespace, utils.GetFullSnapshotLeaseName(etcd)), fullLease); err != nil { logger.Info("Couldn't fetch full snap lease because: " + err.Error()) return ctrl.Result{ @@ -128,7 +127,7 @@ func (lc *CompactionLeaseController) Reconcile(ctx context.Context, req ctrl.Req } deltaLease := &coordinationv1.Lease{} - if err := lc.Get(ctx, kutil.Key(etcd.Namespace, componentlease.GetDeltaSnapshotLeaseName(etcd)), deltaLease); err != nil { + if err := lc.Get(ctx, kutil.Key(etcd.Namespace, utils.GetDeltaSnapshotLeaseName(etcd)), deltaLease); err != nil { logger.Info("Couldn't fetch delta snap lease because: " + err.Error()) return ctrl.Result{ @@ -491,8 +490,8 @@ func getCompactJobCommands(etcd *druidv1alpha1.Etcd) []string { command = append(command, "--data-dir=/var/etcd/data") command = append(command, "--snapstore-temp-directory=/var/etcd/data/tmp") command = append(command, "--enable-snapshot-lease-renewal=true") - command = append(command, "--full-snapshot-lease-name="+componentlease.GetFullSnapshotLeaseName(etcd)) - command = append(command, "--delta-snapshot-lease-name="+componentlease.GetDeltaSnapshotLeaseName(etcd)) + command = append(command, "--full-snapshot-lease-name="+utils.GetFullSnapshotLeaseName(etcd)) + command = append(command, "--delta-snapshot-lease-name="+utils.GetDeltaSnapshotLeaseName(etcd)) var quota int64 = DefaultETCDQuota if etcd.Spec.Etcd.Quota != nil { diff --git a/controllers/compaction_lease_controller_test.go b/controllers/compaction_lease_controller_test.go index d8b1c36ce..aa73b5207 100644 --- a/controllers/compaction_lease_controller_test.go +++ b/controllers/compaction_lease_controller_test.go @@ -757,7 +757,7 @@ func fullLeaseIsCorrectlyReconciled(c client.Client, instance *druidv1alpha1.Etc ctx, cancel := context.WithTimeout(context.TODO(), timeout) defer cancel() req := types.NamespacedName{ - Name: componentlease.GetFullSnapshotLeaseName(instance), + Name: utils.GetFullSnapshotLeaseName(instance), Namespace: instance.Namespace, } @@ -775,7 +775,7 @@ func deltaLeaseIsCorrectlyReconciled(c client.Client, instance *druidv1alpha1.Et ctx, cancel := context.WithTimeout(context.TODO(), timeout) defer cancel() req := types.NamespacedName{ - Name: componentlease.GetDeltaSnapshotLeaseName(instance), + Name: utils.GetDeltaSnapshotLeaseName(instance), Namespace: instance.Namespace, } diff --git a/controllers/etcd_controller.go b/controllers/etcd_controller.go index 41e2d5f52..8ff9f2a91 100644 --- a/controllers/etcd_controller.go +++ b/controllers/etcd_controller.go @@ -29,6 +29,7 @@ import ( componentconfigmap "github.com/gardener/etcd-druid/pkg/component/etcd/configmap" componentlease "github.com/gardener/etcd-druid/pkg/component/etcd/lease" componentservice "github.com/gardener/etcd-druid/pkg/component/etcd/service" + "github.com/gardener/etcd-druid/pkg/component/etcd/statefulset" druidpredicates "github.com/gardener/etcd-druid/pkg/predicate" "github.com/gardener/etcd-druid/pkg/utils" @@ -53,7 +54,6 @@ import ( "k8s.io/apimachinery/pkg/labels" "k8s.io/apimachinery/pkg/runtime" "k8s.io/apimachinery/pkg/types" - errorsutil "k8s.io/apimachinery/pkg/util/errors" "k8s.io/apimachinery/pkg/util/sets" "k8s.io/apimachinery/pkg/util/yaml" "k8s.io/client-go/rest" @@ -370,7 +370,8 @@ func (r *EtcdReconciler) delete(ctx context.Context, etcd *druidv1alpha1.Etcd) ( } } - if waitForStatefulSetCleanup, err := r.removeDependantStatefulset(ctx, logger, etcd); err != nil { + stsDeployer := componentSts.New(r.Client, etcd.Namespace, componentSts.GenerateValues(etcd)) + if err := stsDeployer.Destroy(ctx); err != nil { if err = r.updateEtcdErrorStatus(ctx, etcd, nil, err); err != nil { return ctrl.Result{ Requeue: true, @@ -379,10 +380,6 @@ func (r *EtcdReconciler) delete(ctx context.Context, etcd *druidv1alpha1.Etcd) ( return ctrl.Result{ Requeue: true, }, err - } else if waitForStatefulSetCleanup { - return ctrl.Result{ - RequeueAfter: 30 * time.Second, - }, nil } leaseDeployer := componentlease.New(r.Client, etcd.Namespace, componentlease.GenerateValues(etcd)) @@ -468,236 +465,6 @@ func (r *EtcdReconciler) getPodDisruptionBudgetFromEtcd(etcd *druidv1alpha1.Etcd return nil, fmt.Errorf("missing podDisruptionBudget template file in the charts: %v", pdbPath) } -func (r *EtcdReconciler) reconcileStatefulSet(ctx context.Context, logger logr.Logger, etcd *druidv1alpha1.Etcd, values map[string]interface{}) (*appsv1.StatefulSet, error) { - logger.Info("Reconciling etcd statefulset") - - // If any adoptions are attempted, we should first recheck for deletion with - // an uncached quorum read sometime after listing Machines (see #42639). - canAdoptFunc := RecheckDeletionTimestamp(func() (metav1.Object, error) { - foundEtcd := &druidv1alpha1.Etcd{} - err := r.Get(context.TODO(), types.NamespacedName{Name: etcd.Name, Namespace: etcd.Namespace}, foundEtcd) - if err != nil { - return nil, err - } - - if foundEtcd.GetDeletionTimestamp() != nil { - return nil, fmt.Errorf("%v/%v etcd is marked for deletion", etcd.Namespace, etcd.Name) - } - - if foundEtcd.UID != etcd.UID { - return nil, fmt.Errorf("original %v/%v etcd gone: got uid %v, wanted %v", etcd.Namespace, etcd.Name, foundEtcd.UID, etcd.UID) - } - return foundEtcd, nil - }) - - selector, err := metav1.LabelSelectorAsSelector(etcd.Spec.Selector) - if err != nil { - logger.Error(err, "Error converting etcd selector to selector") - return nil, err - } - dm := NewEtcdDruidRefManager(r.Client, r.Scheme, etcd, selector, etcdGVK, canAdoptFunc) - statefulSets, err := dm.FetchStatefulSet(ctx, etcd) - if err != nil { - logger.Error(err, "Error while fetching StatefulSet") - return nil, err - } - - logger.Info("Claiming existing etcd StatefulSet") - claimedStatefulSets, err := dm.ClaimStatefulsets(ctx, statefulSets) - if err != nil { - return nil, err - } - - if len(claimedStatefulSets) > 0 { - // Keep only 1 statefulset. Delete the rest - for i := 1; i < len(claimedStatefulSets); i++ { - sts := claimedStatefulSets[i] - logger.Info("Found duplicate StatefulSet, deleting it", "statefulset", kutil.Key(sts.Namespace, sts.Name).String()) - if err := r.Delete(ctx, sts); err != nil { - logger.Error(err, "Error in deleting duplicate StatefulSet", "statefulset", kutil.Key(sts.Namespace, sts.Name).String()) - continue - } - } - - // Fetch the updated statefulset - // TODO: (timuthy) Check if this is really needed. - sts := &appsv1.StatefulSet{} - if err := r.Get(ctx, types.NamespacedName{Name: claimedStatefulSets[0].Name, Namespace: claimedStatefulSets[0].Namespace}, sts); err != nil { - return nil, err - } - - // Statefulset is claimed by for this etcd. Just sync the specs - if sts, err = r.syncStatefulSetSpec(ctx, logger, sts, etcd, values); err != nil { - return nil, err - } - - // restart etcd pods in crashloop backoff - selector, err := metav1.LabelSelectorAsSelector(sts.Spec.Selector) - if err != nil { - logger.Error(err, "error converting StatefulSet selector to selector") - return nil, err - } - podList := &corev1.PodList{} - if err := r.List(ctx, podList, client.InNamespace(etcd.Namespace), client.MatchingLabelsSelector{Selector: selector}); err != nil { - return nil, err - } - - for _, pod := range podList.Items { - if utils.IsPodInCrashloopBackoff(pod.Status) { - if err := r.Delete(ctx, &pod); err != nil { - logger.Error(err, fmt.Sprintf("error deleting etcd pod in crashloop: %s/%s", pod.Namespace, pod.Name)) - return nil, err - } - } - } - - sts, err = r.waitUntilStatefulSetReady(ctx, logger, etcd, sts) - return sts, err - } - - // Required statefulset doesn't exist. Create new - sts, err := r.getStatefulSetFromEtcd(etcd, values) - if err != nil { - return nil, err - } - - err = r.Create(ctx, sts) - - // Ignore the precondition violated error, this machine is already updated - // with the desired label. - if err == errorsutil.ErrPreconditionViolated { - logger.Info("StatefulSet %s precondition doesn't hold, skip updating it.", "statefulset", kutil.Key(sts.Namespace, sts.Name).String()) - err = nil - } - if err != nil { - return nil, err - } - - sts, err = r.waitUntilStatefulSetReady(ctx, logger, etcd, sts) - return sts, err -} - -func getContainerMapFromPodTemplateSpec(spec corev1.PodSpec) map[string]corev1.Container { - containers := map[string]corev1.Container{} - for _, c := range spec.Containers { - containers[c.Name] = c - } - return containers -} - -func clusterScaledUpToMultiNode(etcd *druidv1alpha1.Etcd) bool { - if etcd == nil { - return false - } - return etcd.Spec.Replicas != 1 && - // Also consider `0` here because this field was not maintained in earlier releases. - (etcd.Status.Replicas == 0 || - etcd.Status.Replicas == 1) -} - -func (r *EtcdReconciler) syncStatefulSetSpec(ctx context.Context, logger logr.Logger, ss *appsv1.StatefulSet, etcd *druidv1alpha1.Etcd, values map[string]interface{}) (*appsv1.StatefulSet, error) { - decoded, err := r.getStatefulSetFromEtcd(etcd, values) - if err != nil { - return nil, err - } - - if reflect.DeepEqual(ss.Spec, decoded.Spec) { - return ss, nil - } - - ssCopy := ss.DeepCopy() - ssCopy.Spec.Replicas = decoded.Spec.Replicas - ssCopy.Spec.UpdateStrategy = decoded.Spec.UpdateStrategy - - recreateSTS := false - if !reflect.DeepEqual(ssCopy.Spec.Selector, decoded.Spec.Selector) { - recreateSTS = true - } - - // We introduced a peer service for multi-node etcd which must be set - // when the previous single-node StatefulSet still has the client service configured. - if ssCopy.Spec.ServiceName != decoded.Spec.ServiceName { - if clusterScaledUpToMultiNode(etcd) { - recreateSTS = true - } - } - - // Applying suggestions from - containers := getContainerMapFromPodTemplateSpec(ssCopy.Spec.Template.Spec) - for i, c := range decoded.Spec.Template.Spec.Containers { - container, ok := containers[c.Name] - if !ok { - return nil, fmt.Errorf("container with name %s could not be fetched from statefulset %s", c.Name, decoded.Name) - } - // only copy requested resources from the existing stateful set to avoid copying already removed (from the etcd resource) resource limits - decoded.Spec.Template.Spec.Containers[i].Resources.Requests = container.Resources.Requests - } - - ssCopy.Spec.Template = decoded.Spec.Template - - if recreateSTS { - logger.Info("StatefulSet change requires recreation", "statefulset", kutil.Key(ssCopy.Namespace, ssCopy.Name).String()) - err = r.recreateStatefulset(ctx, decoded) - } else { - err = retry.RetryOnConflict(retry.DefaultBackoff, func() error { - return r.Patch(ctx, ssCopy, client.MergeFrom(ss)) - }) - } - - // Ignore the precondition violated error, this machine is already updated - // with the desired label. - if err == errorsutil.ErrPreconditionViolated { - logger.Info("StatefulSet precondition doesn't hold, skip updating it", "statefulset", kutil.Key(ss.Namespace, ss.Name).String()) - err = nil - } - if err != nil { - return nil, err - } - return ssCopy, err -} - -func (r *EtcdReconciler) recreateStatefulset(ctx context.Context, ss *appsv1.StatefulSet) error { - skipDelete := false - err := retry.RetryOnConflict(retry.DefaultBackoff, func() error { - if !skipDelete { - if err := r.Delete(ctx, ss); err != nil && !apierrors.IsNotFound(err) { - return err - } - } - skipDelete = true - return r.Create(ctx, ss) - }) - return err -} - -func (r *EtcdReconciler) getStatefulSetFromEtcd(etcd *druidv1alpha1.Etcd, values map[string]interface{}) (*appsv1.StatefulSet, error) { - var err error - decoded := &appsv1.StatefulSet{} - statefulSetPath := getChartPathForStatefulSet() - chartPath := getChartPath() - renderedChart, err := r.chartApplier.Render(chartPath, etcd.Name, etcd.Namespace, values) - if err != nil { - return nil, err - } - if _, ok := renderedChart.Files()[statefulSetPath]; !ok { - return nil, fmt.Errorf("missing configmap template file in the charts: %v", statefulSetPath) - } - - decoder := yaml.NewYAMLOrJSONDecoder(bytes.NewReader([]byte(renderedChart.Files()[statefulSetPath])), 1024) - if err = decoder.Decode(&decoded); err != nil { - return nil, err - } - return decoded, nil -} - -func decodeObject(renderedChart *chartrenderer.RenderedChart, path string, object interface{}) error { - if content, ok := renderedChart.Files()[path]; ok { - decoder := yaml.NewYAMLOrJSONDecoder(bytes.NewReader([]byte(content)), 1024) - return decoder.Decode(&object) - } - return fmt.Errorf("missing file %s in the rendered chart", path) -} - func (r *EtcdReconciler) reconcileServiceAccount(ctx context.Context, logger logr.Logger, etcd *druidv1alpha1.Etcd, values map[string]interface{}) error { logger.Info("Reconciling serviceaccount") var err error @@ -841,9 +608,10 @@ func (r *EtcdReconciler) reconcileEtcd(ctx context.Context, logger logr.Logger, } val := componentetcd.Values{ - ConfigMap: componentconfigmap.GenerateValues(etcd), - Lease: componentlease.GenerateValues(etcd), - Service: componentservice.GenerateValues(etcd), + ConfigMap: componentconfigmap.GenerateValues(etcd), + Lease: componentlease.GenerateValues(etcd), + Service: componentservice.GenerateValues(etcd), + StatefulSet: statefulset.GenerateValues(etcd), } leaseDeployer := componentlease.New(r.Client, etcd.Namespace, val.Lease) @@ -897,34 +665,6 @@ func (r *EtcdReconciler) reconcileEtcd(ctx context.Context, logger logr.Logger, return &val.Service.ClientServiceName, sts, nil } -func checkEtcdOwnerReference(refs []metav1.OwnerReference, etcd *druidv1alpha1.Etcd) bool { - for _, ownerRef := range refs { - if ownerRef.UID == etcd.UID { - return true - } - } - return false -} - -func checkEtcdAnnotations(annotations map[string]string, etcd metav1.Object) bool { - var ( - ownedBy, ownerType string - ok bool - ) - if annotations == nil { - return false - } - if ownedBy, ok = annotations[common.GardenerOwnedBy]; !ok { - return ok - } - if ownerType, ok = annotations[common.GardenerOwnerType]; !ok { - return ok - } - return ownedBy == fmt.Sprintf("%s/%s", etcd.GetNamespace(), etcd.GetName()) && - ownerType == strings.ToLower(etcdGVK.Kind) - -} - func (r *EtcdReconciler) getMapFromEtcd(im imagevector.ImageVector, etcd *druidv1alpha1.Etcd, val componentetcd.Values, disableEtcdServiceAccountAutomount bool) (map[string]interface{}, error) { statefulsetReplicas := int(etcd.Spec.Replicas) @@ -1207,44 +947,6 @@ func getEtcdImages(im imagevector.ImageVector, etcd *druidv1alpha1.Etcd) (string return etcdImage, etcdBackupImage, nil } -func (r *EtcdReconciler) removeDependantStatefulset(ctx context.Context, logger logr.Logger, etcd *druidv1alpha1.Etcd) (waitForStatefulSetCleanup bool, err error) { - selector, err := metav1.LabelSelectorAsSelector(etcd.Spec.Selector) - if err != nil { - return false, err - } - - statefulSets := &appsv1.StatefulSetList{} - if err = r.List(ctx, statefulSets, client.InNamespace(etcd.Namespace), client.MatchingLabelsSelector{Selector: selector}); err != nil { - return false, err - } - - waitForStatefulSetCleanup = false - - for _, sts := range statefulSets.Items { - if canDeleteStatefulset(&sts, etcd) { - var key = kutil.Key(sts.GetNamespace(), sts.GetName()).String() - logger.Info("Deleting statefulset", "statefulset", key) - if err := r.Delete(ctx, &sts, client.PropagationPolicy(metav1.DeletePropagationBackground)); err != nil { - return false, err - } - - // StatefultSet deletion succeeded. Now we need to wait for it to be cleaned up. - waitForStatefulSetCleanup = true - } - } - - return waitForStatefulSetCleanup, nil -} - -func canDeleteStatefulset(sts *appsv1.StatefulSet, etcd *druidv1alpha1.Etcd) bool { - // Adding check for ownerReference to have the same delete path for statefulset. - // The statefulset with ownerReference will be deleted automatically when etcd is - // delete but we would like to explicitly delete it to maintain uniformity in the - // delete path. - return checkEtcdOwnerReference(sts.GetOwnerReferences(), etcd) || - checkEtcdAnnotations(sts.GetAnnotations(), etcd) -} - func bootstrapReset(etcd *druidv1alpha1.Etcd) { etcd.Status.Members = nil etcd.Status.ClusterSize = pointer.Int32Ptr(etcd.Spec.Replicas) diff --git a/pkg/component/etcd/lease/values_helper.go b/pkg/component/etcd/lease/values_helper.go index 914e56540..31735763a 100644 --- a/pkg/component/etcd/lease/values_helper.go +++ b/pkg/component/etcd/lease/values_helper.go @@ -15,9 +15,8 @@ package lease import ( - "fmt" - druidv1alpha1 "github.com/gardener/etcd-druid/api/v1alpha1" + "github.com/gardener/etcd-druid/pkg/utils" ) // GenerateValues generates `lease.Values` for the lease component with the given parameters. @@ -26,18 +25,8 @@ func GenerateValues(etcd *druidv1alpha1.Etcd) Values { BackupEnabled: etcd.Spec.Backup.Store != nil, EtcdName: etcd.Name, EtcdUID: etcd.UID, - DeltaSnapshotLeaseName: GetDeltaSnapshotLeaseName(etcd), - FullSnapshotLeaseName: GetFullSnapshotLeaseName(etcd), + DeltaSnapshotLeaseName: utils.GetDeltaSnapshotLeaseName(etcd), + FullSnapshotLeaseName: utils.GetFullSnapshotLeaseName(etcd), Replicas: etcd.Spec.Replicas, } } - -// GetDeltaSnapshotLeaseName returns the name of the delta snapshot lease based on the given `etcd` object. -func GetDeltaSnapshotLeaseName(etcd *druidv1alpha1.Etcd) string { - return fmt.Sprintf("%s-delta-snap", etcd.Name) -} - -// GetFullSnapshotLeaseName returns the name of the full snapshot lease based on the given `etcd` object. -func GetFullSnapshotLeaseName(etcd *druidv1alpha1.Etcd) string { - return fmt.Sprintf("%s-full-snap", etcd.Name) -} diff --git a/pkg/component/etcd/statefulset/statefulset.go b/pkg/component/etcd/statefulset/statefulset.go new file mode 100644 index 000000000..82af7b980 --- /dev/null +++ b/pkg/component/etcd/statefulset/statefulset.go @@ -0,0 +1,388 @@ +// Copyright (c) 2022 SAP SE or an SAP affiliate company. All rights reserved. This file is licensed under the Apache Software License, v. 2 except as noted otherwise in the LICENSE file +// +// Licensed 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 statefulset + +import ( + "bytes" + "context" + "fmt" + "reflect" + + druidv1alpha1 "github.com/gardener/etcd-druid/api/v1alpha1" + "github.com/go-logr/logr" + + "github.com/gardener/gardener/pkg/chartrenderer" + gardenercomponent "github.com/gardener/gardener/pkg/operation/botanist/component" + appsv1 "k8s.io/api/apps/v1" + v1 "k8s.io/api/core/v1" + metav1 "k8s.io/apimachinery/pkg/apis/meta/v1" + "k8s.io/apimachinery/pkg/util/yaml" + "k8s.io/client-go/util/retry" + "k8s.io/utils/pointer" + "sigs.k8s.io/controller-runtime/pkg/client" +) + +type component struct { + client client.Client + namespace string + + values Values +} + +func (c *component) Deploy(ctx context.Context) error { + var ( + etcdMainSts = c.emptyStatefulset(c.values.EtcdMainStsName) + //etcdEventSts = c.emptyStatefulset(c.values.EtcdEventStsName) + ) + + if err := c.syncEtcdMainSts(ctx, etcdMainSts); err != nil { + return err + } + + /*if err := c.syncEtcdEventSts(ctx, etcdEventSts); err != nil { + return err + }*/ + + return nil +} + +func (c *component) Destroy(ctx context.Context) error { + if err := c.deleteAllStatefulsets(ctx); err != nil { + return err + } + return nil +} + +func (c *component) syncEtcdMainSts() error { + sts := &appsv1.StatefulSet{ + Spec: appsv1.StatefulSetSpec{ + VolumeClaimTemplates: []v1.PersistentVolumeClaim{ + { + ObjectMeta: metav1.ObjectMeta{ + Name: c.values.VolumeClaimTemplateName, + }, + Spec: v1.PersistentVolumeClaimSpec{ + AccessModes: []v1.PersistentVolumeAccessMode{ + v1.ReadWriteOnce, + }, + StorageClassName: c.values.StorageClass, + Resources: getStorageReq(c.values), + }, + }, + }, + PodManagementPolicy: appsv1.ParallelPodManagement, + UpdateStrategy: appsv1.StatefulSetUpdateStrategy{ + Type: appsv1.RollingUpdateStatefulSetStrategyType, + }, + Replicas: pointer.Int32(c.values.Replicas), + Selector: &metav1.LabelSelector{ + MatchLabels: map[string]string{ + "name": "etcd", + "instance": c.values.EtcdName, + }, + }, + Template: v1.PodTemplateSpec{ + ObjectMeta: metav1.ObjectMeta{ + Annotations: c.values.Annotations, + Labels: c.values.Labels, + }, + Spec: v1.PodSpec{ + HostAliases: []v1.HostAlias{ + { + IP: "127.0.0.1", + Hostnames: []string{c.values.EtcdName + "-local"}, + }, + }, + PriorityClassName: c.values.priorityClassName, + ServiceAccountName: c.values.serviceAccountName, + Affinity: c.values.affinity, + TopologySpreadConstraints: c.values.TopologySpreadConstraints, + Containers: []v1.Container{ + { + Name: "etcd", + Image: c.values.EtcdImage, + ImagePullPolicy: v1.PullIfNotPresent, + Command: c.values.EtcdCommand, + ReadinessProbe: &v1.Probe{ + Handler: v1.Handler{ + Exec: &v1.ExecAction{ + Command: c.values.ReadinessProbeCommand, + }, + }, + InitialDelaySeconds: 15, + PeriodSeconds: 5, + FailureThreshold: 5, + }, + LivenessProbe: &v1.Probe{ + Handler: v1.Handler{ + Exec: &v1.ExecAction{ + Command: c.values.LivenessProbCommand, + }, + }, + InitialDelaySeconds: 15, + PeriodSeconds: 5, + FailureThreshold: 5, + }, + Ports: []v1.ContainerPort{ + { + Name: "server", + Protocol: "TCP", + HostPort: 2380, + }, + { + Name: "client", + Protocol: "TCP", + HostPort: 2379, + }, + }, + Env: []v1.EnvVar{ + { + Name: "ENABLE_TLS", + Value: c.values.EnableClientTLS, + }, + { + Name: "BACKUP_ENDPOINT", + Value: c.values.EnableBackupTLS, + }, + { + Name: "FAIL_BELOW_REVISION_PARAMETER", + Value: enableClientTLS, + }, + }, + VolumeMounts: getSecretVolumeMounts(c.values), + }, + { + Name: "backup-restore", + Image: c.values.BackupImage, + ImagePullPolicy: v1.PullIfNotPresent, + Command: c.values.EtcdBackupCommand, + Ports: []v1.ContainerPort{ + { + Name: "server", + Protocol: "TCP", + HostPort: 8080, + }, + }, + }, + }, + Volumes: getBackupRestoreVolumes(c.values), + }, + }, + }, + } + + if etcd.Spec.Backup.CompactionResources != nil { + job.Spec.Template.Spec.Containers[0].Resources = *etcd.Spec.Backup.CompactionResources + } + + return job, nil +} + +func (c *component) reconcileStatefulSet(ctx context.Context, logger logr.Logger, etcd *druidv1alpha1.Etcd, values map[string]interface{}) (*appsv1.StatefulSet, error) { + // Required statefulset doesn't exist. Create new + sts, err := r.getStatefulSetFromEtcd(etcd, values) + if err != nil { + return nil, err + } + + err = r.Create(ctx, sts) + + // Ignore the precondition violated error, this machine is already updated + // with the desired label. + if err == errorsutil.ErrPreconditionViolated { + logger.Info("StatefulSet %s precondition doesn't hold, skip updating it.", "statefulset", kutil.Key(sts.Namespace, sts.Name).String()) + err = nil + } + if err != nil { + return nil, err + } + + sts, err = r.waitUntilStatefulSetReady(ctx, logger, etcd, sts) + return sts, err +} + +func getContainerMapFromPodTemplateSpec(spec corev1.PodSpec) map[string]corev1.Container { + containers := map[string]corev1.Container{} + for _, c := range spec.Containers { + containers[c.Name] = c + } + return containers +} + +func clusterScaledUpToMultiNode(etcd *druidv1alpha1.Etcd) bool { + if etcd == nil { + return false + } + return etcd.Spec.Replicas != 1 && + // Also consider `0` here because this field was not maintained in earlier releases. + (etcd.Status.Replicas == 0 || + etcd.Status.Replicas == 1) +} + +func (r *EtcdReconciler) syncStatefulSetSpec(ctx context.Context, logger logr.Logger, ss *appsv1.StatefulSet, etcd *druidv1alpha1.Etcd, values map[string]interface{}) (*appsv1.StatefulSet, error) { + decoded, err := r.getStatefulSetFromEtcd(etcd, values) + if err != nil { + return nil, err + } + + if reflect.DeepEqual(ss.Spec, decoded.Spec) { + return ss, nil + } + + ssCopy := ss.DeepCopy() + ssCopy.Spec.Replicas = decoded.Spec.Replicas + ssCopy.Spec.UpdateStrategy = decoded.Spec.UpdateStrategy + + recreateSTS := false + if !reflect.DeepEqual(ssCopy.Spec.Selector, decoded.Spec.Selector) { + recreateSTS = true + } + + // We introduced a peer service for multi-node etcd which must be set + // when the previous single-node StatefulSet still has the client service configured. + if ssCopy.Spec.ServiceName != decoded.Spec.ServiceName { + if clusterScaledUpToMultiNode(etcd) { + recreateSTS = true + } + } + + // Applying suggestions from + containers := getContainerMapFromPodTemplateSpec(ssCopy.Spec.Template.Spec) + for i, c := range decoded.Spec.Template.Spec.Containers { + container, ok := containers[c.Name] + if !ok { + return nil, fmt.Errorf("container with name %s could not be fetched from statefulset %s", c.Name, decoded.Name) + } + // only copy requested resources from the existing stateful set to avoid copying already removed (from the etcd resource) resource limits + decoded.Spec.Template.Spec.Containers[i].Resources.Requests = container.Resources.Requests + } + + ssCopy.Spec.Template = decoded.Spec.Template + + if recreateSTS { + logger.Info("StatefulSet change requires recreation", "statefulset", kutil.Key(ssCopy.Namespace, ssCopy.Name).String()) + err = r.recreateStatefulset(ctx, decoded) + } else { + err = retry.RetryOnConflict(retry.DefaultBackoff, func() error { + return r.Patch(ctx, ssCopy, client.MergeFrom(ss)) + }) + } + + // Ignore the precondition violated error, this machine is already updated + // with the desired label. + if err == errorsutil.ErrPreconditionViolated { + logger.Info("StatefulSet precondition doesn't hold, skip updating it", "statefulset", kutil.Key(ss.Namespace, ss.Name).String()) + err = nil + } + if err != nil { + return nil, err + } + return ssCopy, err +} + +func (r *EtcdReconciler) recreateStatefulset(ctx context.Context, ss *appsv1.StatefulSet) error { + skipDelete := false + err := retry.RetryOnConflict(retry.DefaultBackoff, func() error { + if !skipDelete { + if err := r.Delete(ctx, ss); err != nil && !apierrors.IsNotFound(err) { + return err + } + } + skipDelete = true + return r.Create(ctx, ss) + }) + return err +} + +func (r *EtcdReconciler) getStatefulSetFromEtcd(etcd *druidv1alpha1.Etcd, values map[string]interface{}) (*appsv1.StatefulSet, error) { + var err error + decoded := &appsv1.StatefulSet{} + statefulSetPath := getChartPathForStatefulSet() + chartPath := getChartPath() + renderedChart, err := r.chartApplier.Render(chartPath, etcd.Name, etcd.Namespace, values) + if err != nil { + return nil, err + } + if _, ok := renderedChart.Files()[statefulSetPath]; !ok { + return nil, fmt.Errorf("missing configmap template file in the charts: %v", statefulSetPath) + } + + decoder := yaml.NewYAMLOrJSONDecoder(bytes.NewReader([]byte(renderedChart.Files()[statefulSetPath])), 1024) + if err = decoder.Decode(&decoded); err != nil { + return nil, err + } + return decoded, nil +} + +func decodeObject(renderedChart *chartrenderer.RenderedChart, path string, object interface{}) error { + if content, ok := renderedChart.Files()[path]; ok { + decoder := yaml.NewYAMLOrJSONDecoder(bytes.NewReader([]byte(content)), 1024) + return decoder.Decode(&object) + } + return fmt.Errorf("missing file %s in the rendered chart", path) +} + +func (c *component) deleteAllStatefulsets(ctx context.Context) error { + return c.removeDependantStatefulset(ctx) +} + +func (c *component) removeDependantStatefulset(ctx context.Context) error { + labels := getStsLabels(c.values) + + return client.IgnoreNotFound(c.client.DeleteAllOf(ctx, &appsv1.StatefulSet{}, client.InNamespace(c.namespace), client.MatchingLabels(labels))) +} + +// New creates a new statefulset deployer instance. +func New(c client.Client, namespace string, values Values) gardenercomponent.Deployer { + return &component{ + client: c, + namespace: namespace, + values: values, + } +} + +func (c *component) emptyStatefulset(name string) *appsv1.StatefulSet { + return &appsv1.StatefulSet{ + ObjectMeta: metav1.ObjectMeta{ + Name: name, + Namespace: c.namespace, + }, + } +} + +func getOwnerReferences(val Values) []metav1.OwnerReference { + return []metav1.OwnerReference{ + { + APIVersion: druidv1alpha1.GroupVersion.String(), + Kind: "Etcd", + Name: val.EtcdName, + UID: val.EtcdUID, + Controller: pointer.BoolPtr(true), + BlockOwnerDeletion: pointer.BoolPtr(true), + }, + } +} + +func getStsLabels(val Values) map[string]string { + labels := map[string]string{ + "instance": val.EtcdName, + } + + for k, v := range val.Labels { + labels[k] = v + } + + return labels +} diff --git a/pkg/component/etcd/statefulset/values.go b/pkg/component/etcd/statefulset/values.go new file mode 100644 index 000000000..c3d377f45 --- /dev/null +++ b/pkg/component/etcd/statefulset/values.go @@ -0,0 +1,119 @@ +// Copyright (c) 2022 SAP SE or an SAP affiliate company. All rights reserved. This file is licensed under the Apache Software License, v. 2 except as noted otherwise in the LICENSE file +// +// Licensed 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 statefulset + +import ( + druidv1alpha1 "github.com/gardener/etcd-druid/api/v1alpha1" + corev1 "k8s.io/api/core/v1" + metav1 "k8s.io/apimachinery/pkg/apis/meta/v1" + + "k8s.io/apimachinery/pkg/api/resource" + "k8s.io/apimachinery/pkg/types" +) + +const ( + defaultClientPort = 2379 + defaultServerPort = 2380 +) + +type Values struct { + // EtcdName is the name of the etcd resource. + EtcdName string + // EtcdNameSpace is the namespace of etcd resource + EtcdNameSpace string + // EtcdName is the UID of the etcd resource. + EtcdUID types.UID + // Replicas is the number of ETCD instance that the ETCD cluster will have + Replicas int32 + // Annotations is the annotation provided in ETCD spec + Annotations map[string]string + // Labels is the labels provided in ETCD spec + Labels map[string]string + // BackupImage is the backup restore image + BackupImage string + // EtcdImage is the etcd custom image + EtcdImage string + // priorityClassName is the Priority Class name + priorityClassName string + // serviceAccountName is the service account name + serviceAccountName string + affinity *corev1.Affinity + TopologySpreadConstraints []corev1.TopologySpreadConstraint + + EtcdCommand []string + ReadinessProbeCommand []string + LivenessProbCommand []string + EtcdBackupCommand []string + + EnableClientTLS string + EnablePeerTLS string + EnableBackupTLS string + + FailBelowRevision string + VolumeClaimTemplateName string + + FullSnapLeaseName string + DeltaSnapLeaseName string + + StorageCapacity *resource.Quantity + StorageClass *string + + DefragmentationSchedule *string + FullSnapshotSchedule *string + + EtcdConnectionTimeout *metav1.Duration + ReelectionPeriod *metav1.Duration + + EtcdSnapshotTimeout *metav1.Duration + EtcdDefragTimeout *metav1.Duration + + DeltaSnapshotMemoryLimit *resource.Quantity + + GarbageCollectionPolicy *druidv1alpha1.GarbageCollectionPolicy + GarbageCollectionPeriod *metav1.Duration + + LeaderElection *druidv1alpha1.LeaderElectionSpec + BackupStore *druidv1alpha1.StoreSpec + + EnableProfiling *bool + + DeltaSnapshotPeriod *metav1.Duration + + SnapshotCompression *druidv1alpha1.CompressionSpec + HeartbeatDuration *metav1.Duration + + // Metrics defines the level of detail for exported metrics of etcd, specify 'extensive' to include histogram metrics. + Metrics *druidv1alpha1.MetricsLevel + // Quota defines the etcd DB quota. + Quota *resource.Quantity + // ClientUrlTLS hold the TLS configuration details for Client Communication + ClientUrlTLS *druidv1alpha1.TLSConfig + // PeerUrlTLS hold the TLS configuration details for Peer Communication + PeerUrlTLS *druidv1alpha1.TLSConfig + //ClientServiceName is name of the etcd client service + ClientServiceName string + // ClientPort holds the client port + ClientPort *int32 + //PeerServiceName is name of the etcd peer service + PeerServiceName string + // ServerPort holds the peer port + ServerPort *int32 + // AutoCompactionMode defines the auto-compaction-mode: 'periodic' or 'revision'. + AutoCompactionMode *druidv1alpha1.CompactionMode + //AutoCompactionRetention defines the auto-compaction-retention length for etcd as well as for embedded-Etcd of backup-restore sidecar. + AutoCompactionRetention *string + // ConfigMapName is the name of the configmap that holds the ETCD config + ConfigMapName string +} diff --git a/pkg/component/etcd/statefulset/values_helper.go b/pkg/component/etcd/statefulset/values_helper.go new file mode 100644 index 000000000..983c1cd69 --- /dev/null +++ b/pkg/component/etcd/statefulset/values_helper.go @@ -0,0 +1,523 @@ +// Copyright (c) 2022 SAP SE or an SAP affiliate company. All rights reserved. This file is licensed under the Apache Software License, v.2 except as noted otherwise in the LICENSE file +// +// Licensed 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 statefulset + +import ( + "strconv" + + druidv1alpha1 "github.com/gardener/etcd-druid/api/v1alpha1" + "github.com/gardener/etcd-druid/pkg/utils" + corev1 "k8s.io/api/core/v1" + "k8s.io/utils/pointer" +) + +// GenerateValues generates `statefulset.Values` for the statefulset component with the given parameters. +func GenerateValues(etcd *druidv1alpha1.Etcd, backupImage string, etcdImage string) Values { + enableClientTLS := "false" + if etcd.Spec.Etcd.ClientUrlTLS != nil { + enableClientTLS = "true" + } + + enablePeerTLS := "false" + if etcd.Spec.Etcd.PeerUrlTLS != nil { + enablePeerTLS = "true" + } + + enableBackupTLS := "false" + if etcd.Spec.Backup.TLS != nil { + enableBackupTLS = "true" + } + + volumeClaimTemplateName := etcd.Name + if etcd.Spec.VolumeClaimTemplate != nil && len(*etcd.Spec.VolumeClaimTemplate) != 0 { + volumeClaimTemplateName = *etcd.Spec.VolumeClaimTemplate + } + + values := Values{ + EtcdName: etcd.Name, + EtcdNameSpace: etcd.Namespace, + EtcdUID: etcd.UID, + Replicas: etcd.Spec.Replicas, + Annotations: etcd.Spec.Annotations, + Labels: etcd.Spec.Labels, + BackupImage: backupImage, + EtcdImage: etcdImage, + priorityClassName: *etcd.Spec.PriorityClassName, + serviceAccountName: utils.GetServiceAccountName(etcd), + affinity: etcd.Spec.SchedulingConstraints.Affinity, + TopologySpreadConstraints: etcd.Spec.SchedulingConstraints.TopologySpreadConstraints, + + EnableClientTLS: enableClientTLS, + EnablePeerTLS: enablePeerTLS, + EnableBackupTLS: enableBackupTLS, + VolumeClaimTemplateName: volumeClaimTemplateName, + + FullSnapLeaseName: utils.GetFullSnapshotLeaseName(etcd), + DeltaSnapLeaseName: utils.GetDeltaSnapshotLeaseName(etcd), + + StorageCapacity: etcd.Spec.StorageCapacity, + StorageClass: etcd.Spec.StorageClass, + + ClientUrlTLS: etcd.Spec.Etcd.ClientUrlTLS, + PeerUrlTLS: etcd.Spec.Etcd.PeerUrlTLS, + + EtcdConnectionTimeout: etcd.Spec.Backup.LeaderElection.EtcdConnectionTimeout, + ReelectionPeriod: etcd.Spec.Backup.LeaderElection.ReelectionPeriod, + + LeaderElection: etcd.Spec.Backup.LeaderElection, + + BackupStore: etcd.Spec.Backup.Store, + EnableProfiling: etcd.Spec.Backup.EnableProfiling, + + DeltaSnapshotPeriod: etcd.Spec.Backup.DeltaSnapshotPeriod, + DeltaSnapshotMemoryLimit: etcd.Spec.Backup.DeltaSnapshotMemoryLimit, + + EtcdSnapshotTimeout: etcd.Spec.Backup.EtcdSnapshotTimeout, + EtcdDefragTimeout: etcd.Spec.Etcd.EtcdDefragTimeout, + + GarbageCollectionPolicy: etcd.Spec.Backup.GarbageCollectionPolicy, + GarbageCollectionPeriod: etcd.Spec.Backup.GarbageCollectionPeriod, + + SnapshotCompression: etcd.Spec.Backup.SnapshotCompression, + HeartbeatDuration: etcd.Spec.Etcd.HeartbeatDuration, + + Metrics: etcd.Spec.Etcd.Metrics, + Quota: etcd.Spec.Etcd.Quota, + ClientServiceName: utils.GetClientServiceName(etcd), + ClientPort: etcd.Spec.Etcd.ClientPort, + PeerServiceName: utils.GetPeerServiceName(etcd), + ServerPort: etcd.Spec.Etcd.ServerPort, + AutoCompactionMode: etcd.Spec.Common.AutoCompactionMode, + AutoCompactionRetention: etcd.Spec.Common.AutoCompactionRetention, + ConfigMapName: utils.GetConfigmapName(etcd), + } + + values.EtcdCommand = getEtcdCommand() + values.ReadinessProbeCommand = getReadinessProbeCommand(values) + values.LivenessProbCommand = getLivenessProbeCommand(values) + values.EtcdBackupCommand = getEtcdBackupCommand(values) + + return values +} + +func getEtcdCommand() []string { + command := []string{"" + "/var/etcd/bin/bootstrap.sh"} + + return command +} + +func getReadinessProbeCommand(val Values) []string { + command := []string{"" + "/usr/bin/curl"} + + if val.EnableClientTLS != "true" { + + command = append(command, "--cert") + command = append(command, "/var/etcd/ssl/client/client/tls.crt") + command = append(command, "--key") + command = append(command, "/var/etcd/ssl/client/client/tls.key") + if dataKey := val.ClientUrlTLS.TLSCASecretRef.DataKey; dataKey != nil { + command = append(command, "--cacert") + command = append(command, "/var/etcd/ssl/client/ca/"+*dataKey) + } + + if val.Replicas == 1 { + command = append(command, "https://"+val.EtcdName+"-local:8080/healthz") + } else { + command = append(command, "https://"+val.EtcdName+"-local:2379/health") + } + } else { + if val.Replicas == 1 { + command = append(command, "http://"+val.EtcdName+"-local:8080/healthz") + } else { + command = append(command, "http://"+val.EtcdName+"-local:2379/health") + } + } + return command +} + +func getLivenessProbeCommand(val Values) []string { + command := []string{"" + "/bin/sh"} + command = append(command, "-ec") + command = append(command, "ETCDCTL_API=3") + command = append(command, "etcdctl") + + if val.EnableClientTLS != "true" { + + command = append(command, "--cert=/var/etcd/ssl/client/client/tls.crt") + command = append(command, "--key=/var/etcd/ssl/client/client/tls.key") + if dataKey := val.ClientUrlTLS.TLSCASecretRef.DataKey; dataKey != nil { + command = append(command, "--cacert=/var/etcd/ssl/client/ca/"+*dataKey) + } + command = append(command, "--endpoints=https://"+val.EtcdName+"-local:2379") + } else { + command = append(command, "--endpoints=http://"+val.EtcdName+"-local:2379") + } + command = append(command, "get") + command = append(command, "foo") + command = append(command, "--consistency=s") + return command +} + +func getEtcdBackupCommand(val Values) []string { + command := []string{"" + "etcdbrctl"} + command = append(command, "server") + + command = append(command, "--enable-snapshot-lease-renewal=true") + command = append(command, "--delta-snapshot-lease-name="+val.FullSnapLeaseName) + command = append(command, "--full-snapshot-lease-name="+val.DeltaSnapLeaseName) + + if val.DefragmentationSchedule != nil { + command = append(command, "--defragmentation-schedule="+*val.DefragmentationSchedule) + } + + if val.FullSnapshotSchedule != nil { + command = append(command, "--schedule="+*val.FullSnapshotSchedule) + } + + if val.GarbageCollectionPolicy != nil { + + gbc := string(*val.GarbageCollectionPolicy) + command = append(command, "--garbage-collection-policy="+gbc) + + if gbc == "LimitBased" { + command = append(command, "--max-backups=7") + } + } + + command = append(command, "--data-dir=/var/etcd/data/new.etcd") + + if val.BackupStore != nil { + command = append(command, "--storage-provider="+string(*val.BackupStore.Provider)) + command = append(command, "--store-prefix="+string(*&val.BackupStore.Prefix)) + } + + var quota int64 = 8 * 1024 * 1024 * 1024 // 8Gi + if val.Quota != nil { + quota = val.Quota.Value() + } + + command = append(command, "--embedded-etcd-quota-bytes="+string(quota)) + + command = append(command, "--enable-profiling="+strconv.FormatBool(pointer.BoolDeref(val.EnableProfiling, false))) + + if val.EnableBackupTLS == "true" { + command = append(command, "--cert=/var/etcd/ssl/client/client/tls.crt") + command = append(command, "--key=/var/etcd/ssl/client/client/tls.key") + if dataKey := val.ClientUrlTLS.TLSCASecretRef.DataKey; dataKey != nil { + command = append(command, "--cacert=/var/etcd/ssl/client/ca/"+*dataKey) + } + command = append(command, "--insecure-transport=false") + command = append(command, "--insecure-skip-tls-verify=false") + + command = append(command, "--endpoints=https://"+val.EtcdName+"-local:2379") + + command = append(command, "--server-cert=/var/etcd/ssl/client/server/tls.crt") + command = append(command, "--server-key=/var/etcd/ssl/client/server/tls.key") + } else { + command = append(command, "--endpoints=http://"+val.EtcdName+"-local:2379") + } + + if val.EtcdConnectionTimeout != nil { + command = append(command, "--etcd-connection-timeout="+val.EtcdConnectionTimeout.String()) + } + + if val.DeltaSnapshotPeriod != nil { + command = append(command, "--delta-snapshot-period="+val.DeltaSnapshotPeriod.String()) + } + + var deltaSnapshotMemoryLimit int64 = 100 * 1024 * 1024 // 100Mi + if val.DeltaSnapshotMemoryLimit != nil { + deltaSnapshotMemoryLimit = val.DeltaSnapshotMemoryLimit.Value() + } + + command = append(command, "--delta-snapshot-memory-limit="+string(deltaSnapshotMemoryLimit)) + + if val.GarbageCollectionPeriod != nil { + command = append(command, val.GarbageCollectionPeriod.String()) + } + + if val.SnapshotCompression != nil { + if pointer.BoolPtrDerefOr(val.SnapshotCompression.Enabled, false) { + command = append(command, "--compress-snapshots="+strconv.FormatBool(pointer.BoolPtrDerefOr(val.SnapshotCompression.Enabled, false))) + } + if val.SnapshotCompression.Policy != nil { + command = append(command, "--compression-policy="+string(*val.SnapshotCompression.Policy)) + } + } + + if val.AutoCompactionMode != nil { + command = append(command, "--auto-compaction-mode="+string(*val.AutoCompactionMode)) + } + + if val.AutoCompactionRetention != nil { + command = append(command, "--auto-compaction-retention="+string(*val.AutoCompactionRetention)) + } + + if val.EtcdSnapshotTimeout != nil { + command = append(command, "--etcd-snapshot-timeout="+val.EtcdSnapshotTimeout.String()) + } + + if val.EtcdDefragTimeout != nil { + command = append(command, "--etcd-defrag-timeout="+val.EtcdDefragTimeout.String()) + } + + command = append(command, "--snapstore-temp-directory=/var/etcd/data/temp") + command = append(command, "--etcd-process-name=etcd") + + if heartBeatDuration := val.HeartbeatDuration; heartBeatDuration != nil { + command = append(command, "--k8s-heartbeat-duration="+heartBeatDuration.String()) + } + + if val.LeaderElection != nil { + if val.EtcdConnectionTimeout != nil { + command = append(command, "--etcd-connection-timeout-leader-election="+val.EtcdConnectionTimeout.String()) + } + if val.ReelectionPeriod != nil { + command = append(command, "--reelection-period=", val.ReelectionPeriod.String()) + } + } + + return command +} + +func getStsEnvVar(val Values) []corev1.EnvVar { + var env []corev1.EnvVar + if val.BackupStore == nil { + return env + } + + storeValues := val.BackupStore + + env = append(env, getEnvVarFromValues("STORAGE_CONTAINER", *storeValues.Container)) + env = append(env, getEnvVarFromFields("POD_NAMESPACE", "metadata.namespace")) + + provider, err := utils.StorageProviderFromInfraProvider(val.BackupStore.Provider) + if err != nil { + return env + } + + if provider == "S3" { + env = append(env, getEnvVarFromValues("AWS_APPLICATION_CREDENTIALS", "/root/etcd-backup")) + } + + if provider == "ABS" { + env = append(env, getEnvVarFromValues("AZURE_APPLICATION_CREDENTIALS", "/root/etcd-backup")) + } + + if provider == "GCS" { + env = append(env, getEnvVarFromValues("GOOGLE_APPLICATION_CREDENTIALS", "/root/.gcp/serviceaccount.json")) + } + + if provider == "Swift" { + env = append(env, getEnvVarFromValues("OPENSTACK_APPLICATION_CREDENTIALS", "/root/etcd-backup")) + } + + if provider == "OSS" { + env = append(env, getEnvVarFromValues("ALICLOUD_APPLICATION_CREDENTIALS", "/root/etcd-backup")) + } + + if provider == "ECS" { + env = append(env, getEnvVarFromSecrets("ECS_ENDPOINT", storeValues.SecretRef.Name, "endpoint")) + env = append(env, getEnvVarFromSecrets("ECS_ACCESS_KEY_ID", storeValues.SecretRef.Name, "accessKeyID")) + env = append(env, getEnvVarFromSecrets("ECS_SECRET_ACCESS_KEY", storeValues.SecretRef.Name, "secretAccessKey")) + } + + if provider == "OCS" { + env = append(env, getEnvVarFromValues("OPENSHIFT_APPLICATION_CREDENTIALS", "/root/etcd-backup")) + } + + return env +} + +func getEnvVarFromValues(name, value string) corev1.EnvVar { + return corev1.EnvVar{ + Name: name, + Value: value, + } +} + +func getEnvVarFromFields(name, fieldPath string) corev1.EnvVar { + return corev1.EnvVar{ + Name: name, + ValueFrom: &corev1.EnvVarSource{ + FieldRef: &corev1.ObjectFieldSelector{ + FieldPath: fieldPath, + }, + }, + } +} + +func getEnvVarFromSecrets(name, secretName, secretKey string) corev1.EnvVar { + return corev1.EnvVar{ + Name: name, + ValueFrom: &corev1.EnvVarSource{ + SecretKeyRef: &corev1.SecretKeySelector{ + LocalObjectReference: corev1.LocalObjectReference{ + Name: secretName, + }, + Key: secretKey, + }, + }, + } +} + +func getBackupRestoreVolumes(val Values) []corev1.Volume { + vs := []corev1.Volume{ + { + Name: "etcd-config-file", + VolumeSource: corev1.VolumeSource{ + ConfigMap: &corev1.ConfigMapVolumeSource{ + LocalObjectReference: corev1.LocalObjectReference{ + Name: val.ConfigMapName, + }, + Items: []corev1.KeyToPath{ + { + Key: "etcd.conf.yaml", + Path: "etcd.conf.yaml", + }, + }, + DefaultMode: pointer.Int32(0644), + }, + }, + }, + } + + if val.ClientUrlTLS != nil { + vs = append(vs, corev1.Volume{ + Name: "client-url-ca-etcd", + VolumeSource: corev1.VolumeSource{ + Secret: &corev1.SecretVolumeSource{ + SecretName: val.ClientUrlTLS.TLSCASecretRef.Name, + }, + }, + }, + corev1.Volume{ + Name: "client-url-etcd-server-tls", + VolumeSource: corev1.VolumeSource{ + Secret: &corev1.SecretVolumeSource{ + SecretName: val.ClientUrlTLS.ServerTLSSecretRef.Name, + }, + }, + }, + corev1.Volume{ + Name: "client-url-etcd-client-tls", + VolumeSource: corev1.VolumeSource{ + Secret: &corev1.SecretVolumeSource{ + SecretName: val.ClientUrlTLS.ClientTLSSecretRef.Name, + }, + }, + }) + } + + if val.PeerUrlTLS != nil { + vs = append(vs, corev1.Volume{ + Name: "peer-url-ca-etcd", + VolumeSource: corev1.VolumeSource{ + Secret: &corev1.SecretVolumeSource{ + SecretName: val.PeerUrlTLS.TLSCASecretRef.Name, + }, + }, + }, + corev1.Volume{ + Name: "peer-url-etcd-server-tls", + VolumeSource: corev1.VolumeSource{ + Secret: &corev1.SecretVolumeSource{ + SecretName: val.PeerUrlTLS.ServerTLSSecretRef.Name, + }, + }, + }) + } + + if val.BackupStore == nil { + return vs + } + + storeValues := val.BackupStore + provider, err := utils.StorageProviderFromInfraProvider(storeValues.Provider) + if err != nil { + return vs + } + + if provider == "Local" { + hpt := corev1.HostPathDirectory + vs = append(vs, corev1.Volume{ + Name: "host-storage", + VolumeSource: corev1.VolumeSource{ + HostPath: &corev1.HostPathVolumeSource{ + Path: storeValues.Prefix + "/" + *storeValues.Container, + Type: &hpt, + }, + }, + }) + } + + if provider == "GCS" || provider == "S3" || provider == "OSS" || provider == "ABS" || provider == "Swift" || provider == "OCS" { + vs = append(vs, corev1.Volume{ + Name: "etcd-backup", + VolumeSource: corev1.VolumeSource{ + Secret: &corev1.SecretVolumeSource{ + SecretName: storeValues.SecretRef.Name, + }, + }, + }) + } + + return vs +} + +func getSecretVolumeMounts(val Values) []corev1.VolumeMount { + vms := []corev1.VolumeMount{ + { + Name: val.VolumeClaimTemplateName, + MountPath: "/var/etcd/data/", + }, + } + if val.EnableClientTLS == "true" { + vms = append(vms, corev1.VolumeMount{ + Name: "client-url-ca-etcd", + MountPath: "/var/etcd/ssl/client/ca", + }, corev1.VolumeMount{ + Name: "client-url-etcd-server-tls", + MountPath: "/var/etcd/ssl/client/server", + }, corev1.VolumeMount{ + Name: "client-url-etcd-client-tls", + MountPath: "/var/etcd/ssl/client/client", + }) + } + + if val.EnablePeerTLS == "true" { + vms = append(vms, corev1.VolumeMount{ + Name: "peer-url-ca-etcd", + MountPath: "/var/etcd/ssl/peer/ca", + }, corev1.VolumeMount{ + Name: "peer-url-etcd-server-tls", + MountPath: "/var/etcd/ssl/peer/server", + }) + } + + return vms +} + +func getStorageReq(val Values) corev1.ResourceRequirements { + if val.StorageCapacity != nil { + return corev1.ResourceRequirements{ + Requests: corev1.ResourceList{ + "storage": *val.StorageCapacity, + }, + } + } + + return corev1.ResourceRequirements{} +} diff --git a/pkg/component/etcd/values.go b/pkg/component/etcd/values.go index cc95326ba..fbf9c83ad 100644 --- a/pkg/component/etcd/values.go +++ b/pkg/component/etcd/values.go @@ -18,11 +18,13 @@ import ( "github.com/gardener/etcd-druid/pkg/component/etcd/configmap" "github.com/gardener/etcd-druid/pkg/component/etcd/lease" "github.com/gardener/etcd-druid/pkg/component/etcd/service" + "github.com/gardener/etcd-druid/pkg/component/etcd/statefulset" ) // Values contains all values relevant for deploying etcd components. type Values struct { - ConfigMap *configmap.Values - Service service.Values - Lease lease.Values + ConfigMap *configmap.Values + Service service.Values + Lease lease.Values + StatefulSet statefulset.Values } diff --git a/pkg/utils/names.go b/pkg/utils/names.go index 8ad48319a..575a5523d 100644 --- a/pkg/utils/names.go +++ b/pkg/utils/names.go @@ -54,3 +54,13 @@ func GetJobName(etcd *druidv1alpha1.Etcd) string { func GetOrdinalPodName(etcd *druidv1alpha1.Etcd, order int) string { return fmt.Sprintf("%s-%d", etcd.Name, order) } + +// GetDeltaSnapshotLeaseName returns the name of the delta snapshot lease based on the given `etcd` object. +func GetDeltaSnapshotLeaseName(etcd *druidv1alpha1.Etcd) string { + return fmt.Sprintf("%s-delta-snap", etcd.Name) +} + +// GetFullSnapshotLeaseName returns the name of the full snapshot lease based on the given `etcd` object. +func GetFullSnapshotLeaseName(etcd *druidv1alpha1.Etcd) string { + return fmt.Sprintf("%s-full-snap", etcd.Name) +}