run async worker with no intervals

Signed-off-by: Garrybest <garrybest@foxmail.com>
This commit is contained in:
Garrybest 2021-10-18 10:58:49 +08:00
parent c018f79695
commit 08fa785670
6 changed files with 12 additions and 19 deletions

View File

@ -4,7 +4,6 @@ import (
"context" "context"
"fmt" "fmt"
"os" "os"
"time"
corev1 "k8s.io/api/core/v1" corev1 "k8s.io/api/core/v1"
apierrors "k8s.io/apimachinery/pkg/api/errors" apierrors "k8s.io/apimachinery/pkg/api/errors"
@ -52,7 +51,7 @@ func (d *ClusterDetector) Start(ctx context.Context) error {
d.stopCh = ctx.Done() d.stopCh = ctx.Done()
d.EventHandler = informermanager.NewHandlerOnEvents(d.OnAdd, d.OnUpdate, d.OnDelete) d.EventHandler = informermanager.NewHandlerOnEvents(d.OnAdd, d.OnUpdate, d.OnDelete)
d.Processor = util.NewAsyncWorker("cluster-api cluster detector", time.Second, ClusterWideKeyFunc, d.Reconcile) d.Processor = util.NewAsyncWorker("cluster-api cluster detector", ClusterWideKeyFunc, d.Reconcile)
d.Processor.Run(1, d.stopCh) d.Processor.Run(1, d.stopCh)
d.discoveryCluster() d.discoveryCluster()

View File

@ -5,7 +5,6 @@ import (
"fmt" "fmt"
"reflect" "reflect"
"sync" "sync"
"time"
discoveryv1beta1 "k8s.io/api/discovery/v1beta1" discoveryv1beta1 "k8s.io/api/discovery/v1beta1"
apierrors "k8s.io/apimachinery/pkg/api/errors" apierrors "k8s.io/apimachinery/pkg/api/errors"
@ -106,7 +105,7 @@ func (c *ServiceExportController) SetupWithManager(mgr controllerruntime.Manager
// RunWorkQueue initializes worker and run it, worker will process resource asynchronously. // RunWorkQueue initializes worker and run it, worker will process resource asynchronously.
func (c *ServiceExportController) RunWorkQueue() { func (c *ServiceExportController) RunWorkQueue() {
c.worker = util.NewAsyncWorker("service-export", time.Second, nil, c.syncServiceExportOrEndpointSlice) c.worker = util.NewAsyncWorker("service-export", nil, c.syncServiceExportOrEndpointSlice)
c.worker.Run(c.WorkerNumber, c.StopChan) c.worker.Run(c.WorkerNumber, c.StopChan)
} }

View File

@ -5,7 +5,6 @@ import (
"encoding/json" "encoding/json"
"fmt" "fmt"
"reflect" "reflect"
"time"
apierrors "k8s.io/apimachinery/pkg/api/errors" apierrors "k8s.io/apimachinery/pkg/api/errors"
"k8s.io/apimachinery/pkg/api/meta" "k8s.io/apimachinery/pkg/api/meta"
@ -108,7 +107,7 @@ func (c *WorkStatusController) getEventHandler() cache.ResourceEventHandler {
// RunWorkQueue initializes worker and run it, worker will process resource asynchronously. // RunWorkQueue initializes worker and run it, worker will process resource asynchronously.
func (c *WorkStatusController) RunWorkQueue() { func (c *WorkStatusController) RunWorkQueue() {
c.worker = util.NewAsyncWorker("work-status", time.Second, generateKey, c.syncWorkStatus) c.worker = util.NewAsyncWorker("work-status", generateKey, c.syncWorkStatus)
c.worker.Run(c.WorkerNumber, c.StopChan) c.worker.Run(c.WorkerNumber, c.StopChan)
} }

View File

@ -131,7 +131,7 @@ func NewScheduler(dynamicClient dynamic.Interface, karmadaClient karmadaclientse
if opts.EnableSchedulerEstimator { if opts.EnableSchedulerEstimator {
sched.schedulerEstimatorCache = estimatorclient.NewSchedulerEstimatorCache() sched.schedulerEstimatorCache = estimatorclient.NewSchedulerEstimatorCache()
sched.schedulerEstimatorPort = opts.SchedulerEstimatorPort sched.schedulerEstimatorPort = opts.SchedulerEstimatorPort
sched.schedulerEstimatorWorker = util.NewAsyncWorker("scheduler-estimator", 0, nil, sched.reconcileEstimatorConnection) sched.schedulerEstimatorWorker = util.NewAsyncWorker("scheduler-estimator", nil, sched.reconcileEstimatorConnection)
schedulerEstimator := estimatorclient.NewSchedulerEstimator(sched.schedulerEstimatorCache, opts.SchedulerEstimatorTimeout.Duration) schedulerEstimator := estimatorclient.NewSchedulerEstimator(sched.schedulerEstimatorCache, opts.SchedulerEstimatorTimeout.Duration)
estimatorclient.RegisterSchedulerEstimator(schedulerEstimator) estimatorclient.RegisterSchedulerEstimator(schedulerEstimator)
} }

View File

@ -81,9 +81,9 @@ func (d *ResourceDetector) Start(ctx context.Context) error {
d.stopCh = ctx.Done() d.stopCh = ctx.Done()
// setup policy reconcile worker // setup policy reconcile worker
d.policyReconcileWorker = util.NewAsyncWorker("propagationPolicy reconciler", 1*time.Millisecond, ClusterWideKeyFunc, d.ReconcilePropagationPolicy) d.policyReconcileWorker = util.NewAsyncWorker("propagationPolicy reconciler", ClusterWideKeyFunc, d.ReconcilePropagationPolicy)
d.policyReconcileWorker.Run(1, d.stopCh) d.policyReconcileWorker.Run(1, d.stopCh)
d.clusterPolicyReconcileWorker = util.NewAsyncWorker("clusterPropagationPolicy reconciler", time.Microsecond, ClusterWideKeyFunc, d.ReconcileClusterPropagationPolicy) d.clusterPolicyReconcileWorker = util.NewAsyncWorker("clusterPropagationPolicy reconciler", ClusterWideKeyFunc, d.ReconcileClusterPropagationPolicy)
d.clusterPolicyReconcileWorker.Run(1, d.stopCh) d.clusterPolicyReconcileWorker.Run(1, d.stopCh)
// watch and enqueue PropagationPolicy changes. // watch and enqueue PropagationPolicy changes.
@ -107,7 +107,7 @@ func (d *ResourceDetector) Start(ctx context.Context) error {
d.clusterPropagationPolicyLister = d.InformerManager.Lister(clusterPropagationPolicyGVR) d.clusterPropagationPolicyLister = d.InformerManager.Lister(clusterPropagationPolicyGVR)
// setup binding reconcile worker // setup binding reconcile worker
d.bindingReconcileWorker = util.NewAsyncWorker("resourceBinding reconciler", time.Microsecond, ClusterWideKeyFunc, d.ReconcileResourceBinding) d.bindingReconcileWorker = util.NewAsyncWorker("resourceBinding reconciler", ClusterWideKeyFunc, d.ReconcileResourceBinding)
d.bindingReconcileWorker.Run(1, d.stopCh) d.bindingReconcileWorker.Run(1, d.stopCh)
// watch and enqueue ResourceBinding changes. // watch and enqueue ResourceBinding changes.
@ -130,7 +130,7 @@ func (d *ResourceDetector) Start(ctx context.Context) error {
d.InformerManager.ForResource(clusterResourceBindingGVR, clusterBindingHandler) d.InformerManager.ForResource(clusterResourceBindingGVR, clusterBindingHandler)
d.EventHandler = informermanager.NewFilteringHandlerOnAllEvents(d.EventFilter, d.OnAdd, d.OnUpdate, d.OnDelete) d.EventHandler = informermanager.NewFilteringHandlerOnAllEvents(d.EventFilter, d.OnAdd, d.OnUpdate, d.OnDelete)
d.Processor = util.NewAsyncWorker("resource detector", time.Microsecond, ClusterWideKeyFunc, d.Reconcile) d.Processor = util.NewAsyncWorker("resource detector", ClusterWideKeyFunc, d.Reconcile)
d.Processor.Run(1, d.stopCh) d.Processor.Run(1, d.stopCh)
go d.discoverResources(30 * time.Second) go d.discoverResources(30 * time.Second)

View File

@ -1,12 +1,11 @@
package util package util
import ( import (
"time" "k8s.io/client-go/util/workqueue"
"k8s.io/klog/v2"
"k8s.io/apimachinery/pkg/runtime" "k8s.io/apimachinery/pkg/runtime"
"k8s.io/apimachinery/pkg/util/wait" "k8s.io/apimachinery/pkg/util/wait"
"k8s.io/client-go/util/workqueue"
"k8s.io/klog/v2"
) )
const ( const (
@ -48,17 +47,14 @@ type asyncWorker struct {
reconcileFunc ReconcileFunc reconcileFunc ReconcileFunc
// queue allowing parallel processing of resources. // queue allowing parallel processing of resources.
queue workqueue.RateLimitingInterface queue workqueue.RateLimitingInterface
// interval is the interval for process object in the queue.
interval time.Duration
} }
// NewAsyncWorker returns a asyncWorker which can process resource periodic. // NewAsyncWorker returns a asyncWorker which can process resource periodic.
func NewAsyncWorker(name string, interval time.Duration, keyFunc KeyFunc, reconcileFunc ReconcileFunc) AsyncWorker { func NewAsyncWorker(name string, keyFunc KeyFunc, reconcileFunc ReconcileFunc) AsyncWorker {
return &asyncWorker{ return &asyncWorker{
keyFunc: keyFunc, keyFunc: keyFunc,
reconcileFunc: reconcileFunc, reconcileFunc: reconcileFunc,
queue: workqueue.NewNamedRateLimitingQueue(workqueue.DefaultControllerRateLimiter(), name), queue: workqueue.NewNamedRateLimitingQueue(workqueue.DefaultControllerRateLimiter(), name),
interval: interval,
} }
} }
@ -113,7 +109,7 @@ func (w *asyncWorker) worker() {
func (w *asyncWorker) Run(workerNumber int, stopChan <-chan struct{}) { func (w *asyncWorker) Run(workerNumber int, stopChan <-chan struct{}) {
for i := 0; i < workerNumber; i++ { for i := 0; i < workerNumber; i++ {
go wait.Until(w.worker, w.interval, stopChan) go wait.Until(w.worker, 0, stopChan)
} }
// Ensure all goroutines are cleaned up when the stop channel closes // Ensure all goroutines are cleaned up when the stop channel closes
go func() { go func() {