mirror of
https://github.com/minio/minio.git
synced 2025-11-09 13:39:46 -05:00
simplify usage of mutexes and atomic constants (#9501)
This commit is contained in:
@@ -688,13 +688,15 @@ func (a adminAPIHandlers) HealHandler(w http.ResponseWriter, r *http.Request) {
|
||||
|
||||
// Define a closure to start sending whitespace to client
|
||||
// after 10s unless a response item comes in
|
||||
keepConnLive := func(w http.ResponseWriter, respCh chan healResp) {
|
||||
keepConnLive := func(w http.ResponseWriter, r *http.Request, respCh chan healResp) {
|
||||
ticker := time.NewTicker(time.Second * 10)
|
||||
defer ticker.Stop()
|
||||
started := false
|
||||
forLoop:
|
||||
for {
|
||||
select {
|
||||
case <-r.Context().Done():
|
||||
return
|
||||
case <-ticker.C:
|
||||
if !started {
|
||||
// Start writing response to client
|
||||
@@ -790,7 +792,7 @@ func (a adminAPIHandlers) HealHandler(w http.ResponseWriter, r *http.Request) {
|
||||
respCh <- hr
|
||||
}()
|
||||
case hip.clientToken == "":
|
||||
nh := newHealSequence(hip.bucket, hip.objPrefix, handlers.GetSourceIP(r), numDisks, hip.hs, hip.forceStart)
|
||||
nh := newHealSequence(GlobalContext, hip.bucket, hip.objPrefix, handlers.GetSourceIP(r), numDisks, hip.hs, hip.forceStart)
|
||||
go func() {
|
||||
respBytes, apiErr, errMsg := globalAllHealState.LaunchNewHealSequence(nh)
|
||||
hr := healResp{respBytes, apiErr, errMsg}
|
||||
@@ -801,7 +803,7 @@ func (a adminAPIHandlers) HealHandler(w http.ResponseWriter, r *http.Request) {
|
||||
// Due to the force-starting functionality, the Launch
|
||||
// call above can take a long time - to keep the
|
||||
// connection alive, we start sending whitespace
|
||||
keepConnLive(w, respCh)
|
||||
keepConnLive(w, r, respCh)
|
||||
}
|
||||
|
||||
func (a adminAPIHandlers) BackgroundHealStatusHandler(w http.ResponseWriter, r *http.Request) {
|
||||
|
||||
@@ -60,9 +60,8 @@ const (
|
||||
)
|
||||
|
||||
var (
|
||||
errHealIdleTimeout = fmt.Errorf("healing results were not consumed for too long")
|
||||
errHealPushStopNDiscard = fmt.Errorf("heal push stopped due to heal stop signal")
|
||||
errHealStopSignalled = fmt.Errorf("heal stop signaled")
|
||||
errHealIdleTimeout = fmt.Errorf("healing results were not consumed for too long")
|
||||
errHealStopSignalled = fmt.Errorf("heal stop signaled")
|
||||
|
||||
errFnHealFromAPIErr = func(ctx context.Context, err error) error {
|
||||
apiErr := toAPIError(ctx, err)
|
||||
@@ -73,10 +72,6 @@ var (
|
||||
|
||||
// healSequenceStatus - accumulated status of the heal sequence
|
||||
type healSequenceStatus struct {
|
||||
// lock to update this structure as it is concurrently
|
||||
// accessed
|
||||
updateLock *sync.RWMutex
|
||||
|
||||
// summary and detail for failures
|
||||
Summary healStatusSummary `json:"Summary"`
|
||||
FailureDetail string `json:"Detail,omitempty"`
|
||||
@@ -114,11 +109,9 @@ func initHealState() *allHealState {
|
||||
func (ahs *allHealState) periodicHealSeqsClean(ctx context.Context) {
|
||||
// Launch clean-up routine to remove this heal sequence (after
|
||||
// it ends) from the global state after timeout has elapsed.
|
||||
ticker := time.NewTicker(time.Minute * 5)
|
||||
defer ticker.Stop()
|
||||
for {
|
||||
select {
|
||||
case <-ticker.C:
|
||||
case <-time.After(time.Minute * 5):
|
||||
now := UTCNow()
|
||||
ahs.Lock()
|
||||
for path, h := range ahs.healSeqMap {
|
||||
@@ -202,9 +195,7 @@ func (ahs *allHealState) LaunchNewHealSequence(h *healSequence) (
|
||||
existsAndLive := false
|
||||
he, exists := ahs.getHealSequence(h.path)
|
||||
if exists {
|
||||
if !he.hasEnded() || len(he.currentStatus.Items) > 0 {
|
||||
existsAndLive = true
|
||||
}
|
||||
existsAndLive = !he.hasEnded()
|
||||
}
|
||||
|
||||
if existsAndLive {
|
||||
@@ -277,8 +268,8 @@ func (ahs *allHealState) PopHealStatusJSON(path string,
|
||||
}
|
||||
|
||||
// Take lock to access and update the heal-sequence
|
||||
h.currentStatus.updateLock.Lock()
|
||||
defer h.currentStatus.updateLock.Unlock()
|
||||
h.mutex.Lock()
|
||||
defer h.mutex.Unlock()
|
||||
|
||||
numItems := len(h.currentStatus.Items)
|
||||
|
||||
@@ -289,20 +280,18 @@ func (ahs *allHealState) PopHealStatusJSON(path string,
|
||||
lastResultIndex = h.currentStatus.Items[numItems-1].ResultIndex
|
||||
}
|
||||
|
||||
// After sending status to client, and before relinquishing
|
||||
// the updateLock, reset Item to nil and record the result
|
||||
// index sent to the client.
|
||||
defer func(i int64) {
|
||||
h.lastSentResultIndex = i
|
||||
h.currentStatus.Items = nil
|
||||
}(lastResultIndex)
|
||||
h.lastSentResultIndex = lastResultIndex
|
||||
|
||||
jbytes, err := json.Marshal(h.currentStatus)
|
||||
if err != nil {
|
||||
h.currentStatus.Items = nil
|
||||
|
||||
logger.LogIf(h.ctx, err)
|
||||
return nil, ErrInternalError
|
||||
}
|
||||
|
||||
h.currentStatus.Items = nil
|
||||
|
||||
return jbytes, ErrNone
|
||||
}
|
||||
|
||||
@@ -352,9 +341,8 @@ type healSequence struct {
|
||||
// completed
|
||||
traverseAndHealDoneCh chan error
|
||||
|
||||
// channel to signal heal sequence to stop (e.g. from the
|
||||
// heal-stop API)
|
||||
stopSignalCh chan struct{}
|
||||
// canceler to cancel heal sequence.
|
||||
cancelCtx context.CancelFunc
|
||||
|
||||
// the last result index sent to client
|
||||
lastSentResultIndex int64
|
||||
@@ -380,12 +368,12 @@ type healSequence struct {
|
||||
|
||||
// NewHealSequence - creates healSettings, assumes bucket and
|
||||
// objPrefix are already validated.
|
||||
func newHealSequence(bucket, objPrefix, clientAddr string,
|
||||
func newHealSequence(ctx context.Context, bucket, objPrefix, clientAddr string,
|
||||
numDisks int, hs madmin.HealOpts, forceStart bool) *healSequence {
|
||||
|
||||
reqInfo := &logger.ReqInfo{RemoteHost: clientAddr, API: "Heal", BucketName: bucket}
|
||||
reqInfo.AppendTags("prefix", objPrefix)
|
||||
ctx := logger.SetReqInfo(GlobalContext, reqInfo)
|
||||
ctx, cancel := context.WithCancel(logger.SetReqInfo(ctx, reqInfo))
|
||||
|
||||
return &healSequence{
|
||||
respCh: make(chan healResult),
|
||||
@@ -402,10 +390,9 @@ func newHealSequence(bucket, objPrefix, clientAddr string,
|
||||
Summary: healNotStartedStatus,
|
||||
HealSettings: hs,
|
||||
NumDisks: numDisks,
|
||||
updateLock: &sync.RWMutex{},
|
||||
},
|
||||
traverseAndHealDoneCh: make(chan error),
|
||||
stopSignalCh: make(chan struct{}),
|
||||
cancelCtx: cancel,
|
||||
ctx: ctx,
|
||||
scannedItemsMap: make(map[madmin.HealItemType]int64),
|
||||
healedItemsMap: make(map[madmin.HealItemType]int64),
|
||||
@@ -488,7 +475,7 @@ func (h *healSequence) gethealFailedItemsMap() map[string]int64 {
|
||||
// external signal)
|
||||
func (h *healSequence) isQuitting() bool {
|
||||
select {
|
||||
case <-h.stopSignalCh:
|
||||
case <-h.ctx.Done():
|
||||
return true
|
||||
default:
|
||||
return false
|
||||
@@ -497,19 +484,15 @@ func (h *healSequence) isQuitting() bool {
|
||||
|
||||
// check if the heal sequence has ended
|
||||
func (h *healSequence) hasEnded() bool {
|
||||
h.currentStatus.updateLock.RLock()
|
||||
summary := h.currentStatus.Summary
|
||||
h.currentStatus.updateLock.RUnlock()
|
||||
return summary == healStoppedStatus || summary == healFinishedStatus
|
||||
h.mutex.RLock()
|
||||
ended := len(h.currentStatus.Items) == 0 || h.currentStatus.Summary == healStoppedStatus || h.currentStatus.Summary == healFinishedStatus
|
||||
h.mutex.RUnlock()
|
||||
return ended
|
||||
}
|
||||
|
||||
// stops the heal sequence - safe to call multiple times.
|
||||
func (h *healSequence) stop() {
|
||||
select {
|
||||
case <-h.stopSignalCh:
|
||||
default:
|
||||
close(h.stopSignalCh)
|
||||
}
|
||||
h.cancelCtx()
|
||||
}
|
||||
|
||||
// pushHealResultItem - pushes a heal result item for consumption in
|
||||
@@ -536,29 +519,27 @@ func (h *healSequence) pushHealResultItem(r madmin.HealResultItem) error {
|
||||
|
||||
var itemsLen int
|
||||
for {
|
||||
h.currentStatus.updateLock.Lock()
|
||||
h.mutex.Lock()
|
||||
itemsLen = len(h.currentStatus.Items)
|
||||
if itemsLen == maxUnconsumedHealResultItems {
|
||||
// unlock and wait to check again if we can push
|
||||
h.currentStatus.updateLock.Unlock()
|
||||
|
||||
// wait for a second, or quit if an external
|
||||
// stop signal is received or the
|
||||
// unconsumedTimer fires.
|
||||
select {
|
||||
// Check after a second
|
||||
case <-time.After(time.Second):
|
||||
h.mutex.Unlock()
|
||||
continue
|
||||
|
||||
case <-h.stopSignalCh:
|
||||
case <-h.ctx.Done():
|
||||
h.mutex.Unlock()
|
||||
// discard result and return.
|
||||
return errHealPushStopNDiscard
|
||||
return errHealStopSignalled
|
||||
|
||||
// Timeout if no results consumed for too
|
||||
// long.
|
||||
// Timeout if no results consumed for too long.
|
||||
case <-unconsumedTimer.C:
|
||||
h.mutex.Unlock()
|
||||
return errHealIdleTimeout
|
||||
|
||||
}
|
||||
}
|
||||
break
|
||||
@@ -575,13 +556,7 @@ func (h *healSequence) pushHealResultItem(r madmin.HealResultItem) error {
|
||||
h.currentStatus.Items = append(h.currentStatus.Items, r)
|
||||
|
||||
// release lock
|
||||
h.currentStatus.updateLock.Unlock()
|
||||
|
||||
// This is a "safe" point for the heal sequence to quit if
|
||||
// signaled externally.
|
||||
if h.isQuitting() {
|
||||
return errHealStopSignalled
|
||||
}
|
||||
h.mutex.Unlock()
|
||||
|
||||
return nil
|
||||
}
|
||||
@@ -595,10 +570,10 @@ func (h *healSequence) pushHealResultItem(r madmin.HealResultItem) error {
|
||||
// the heal-sequence.
|
||||
func (h *healSequence) healSequenceStart() {
|
||||
// Set status as running
|
||||
h.currentStatus.updateLock.Lock()
|
||||
h.mutex.Lock()
|
||||
h.currentStatus.Summary = healRunningStatus
|
||||
h.currentStatus.StartTime = UTCNow()
|
||||
h.currentStatus.updateLock.Unlock()
|
||||
h.mutex.Unlock()
|
||||
|
||||
if h.sourceCh == nil {
|
||||
go h.traverseAndHeal()
|
||||
@@ -608,25 +583,27 @@ func (h *healSequence) healSequenceStart() {
|
||||
|
||||
select {
|
||||
case err, ok := <-h.traverseAndHealDoneCh:
|
||||
if !ok {
|
||||
return
|
||||
}
|
||||
h.mutex.Lock()
|
||||
h.endTime = UTCNow()
|
||||
h.currentStatus.updateLock.Lock()
|
||||
defer h.currentStatus.updateLock.Unlock()
|
||||
// Heal traversal is complete.
|
||||
if ok {
|
||||
if err == nil {
|
||||
// heal traversal succeeded.
|
||||
h.currentStatus.Summary = healFinishedStatus
|
||||
} else {
|
||||
// heal traversal had an error.
|
||||
h.currentStatus.Summary = healStoppedStatus
|
||||
h.currentStatus.FailureDetail = err.Error()
|
||||
} else {
|
||||
// heal traversal succeeded.
|
||||
h.currentStatus.Summary = healFinishedStatus
|
||||
}
|
||||
|
||||
case <-h.stopSignalCh:
|
||||
h.mutex.Unlock()
|
||||
case <-h.ctx.Done():
|
||||
h.mutex.Lock()
|
||||
h.endTime = UTCNow()
|
||||
h.currentStatus.updateLock.Lock()
|
||||
h.currentStatus.Summary = healStoppedStatus
|
||||
h.currentStatus.FailureDetail = errHealStopSignalled.Error()
|
||||
h.currentStatus.updateLock.Unlock()
|
||||
h.mutex.Unlock()
|
||||
|
||||
// drain traverse channel so the traversal
|
||||
// go-routine does not leak.
|
||||
@@ -654,13 +631,20 @@ func (h *healSequence) queueHealTask(source healSource, healType madmin.HealItem
|
||||
select {
|
||||
case res := <-h.respCh:
|
||||
if !h.reportProgress {
|
||||
// Object might have been deleted, by the time heal
|
||||
// was attempted, we should ignore this object and
|
||||
// return success.
|
||||
if isErrObjectNotFound(res.err) {
|
||||
return nil
|
||||
}
|
||||
|
||||
h.mutex.Lock()
|
||||
defer h.mutex.Unlock()
|
||||
|
||||
// Progress is not reported in case of background heal processing.
|
||||
// Instead we increment relevant counter based on the heal result
|
||||
// for prometheus reporting.
|
||||
if res.err != nil && !isErrObjectNotFound(res.err) {
|
||||
if res.err != nil {
|
||||
for _, d := range res.result.After.Drives {
|
||||
// For failed items we report the endpoint and drive state
|
||||
// This will help users take corrective actions for drives
|
||||
@@ -670,7 +654,9 @@ func (h *healSequence) queueHealTask(source healSource, healType madmin.HealItem
|
||||
// Only object type reported for successful healing
|
||||
h.healedItemsMap[res.result.Type]++
|
||||
}
|
||||
return nil
|
||||
|
||||
// Report caller of any failure
|
||||
return res.err
|
||||
}
|
||||
res.result.Type = healType
|
||||
if res.err != nil {
|
||||
@@ -688,10 +674,7 @@ func (h *healSequence) queueHealTask(source healSource, healType madmin.HealItem
|
||||
return h.pushHealResultItem(res.result)
|
||||
case <-h.ctx.Done():
|
||||
return nil
|
||||
case <-h.traverseAndHealDoneCh:
|
||||
return nil
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
func (h *healSequence) healItemsFromSourceCh() error {
|
||||
@@ -702,7 +685,10 @@ func (h *healSequence) healItemsFromSourceCh() error {
|
||||
|
||||
for {
|
||||
select {
|
||||
case source := <-h.sourceCh:
|
||||
case source, ok := <-h.sourceCh:
|
||||
if !ok {
|
||||
return nil
|
||||
}
|
||||
var itemType madmin.HealItemType
|
||||
switch {
|
||||
case source.path == nopHeal:
|
||||
@@ -721,8 +707,6 @@ func (h *healSequence) healItemsFromSourceCh() error {
|
||||
|
||||
h.scannedItemsMap[itemType]++
|
||||
h.lastHealActivity = UTCNow()
|
||||
case <-h.traverseAndHealDoneCh:
|
||||
return nil
|
||||
case <-h.ctx.Done():
|
||||
return nil
|
||||
}
|
||||
@@ -763,13 +747,7 @@ func (h *healSequence) healItems(bucketsOnly bool) error {
|
||||
// two objects.
|
||||
func (h *healSequence) traverseAndHeal() {
|
||||
bucketsOnly := false // Heals buckets and objects also.
|
||||
if err := h.healItems(bucketsOnly); err != nil {
|
||||
if h.isQuitting() {
|
||||
err = errHealStopSignalled
|
||||
}
|
||||
h.traverseAndHealDoneCh <- err
|
||||
}
|
||||
|
||||
h.traverseAndHealDoneCh <- h.healItems(bucketsOnly)
|
||||
close(h.traverseAndHealDoneCh)
|
||||
}
|
||||
|
||||
|
||||
@@ -29,6 +29,7 @@ import (
|
||||
"os"
|
||||
"strings"
|
||||
"sync"
|
||||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"github.com/djherbis/atime"
|
||||
@@ -37,7 +38,6 @@ import (
|
||||
"github.com/minio/minio/cmd/logger"
|
||||
"github.com/minio/minio/pkg/disk"
|
||||
"github.com/minio/sio"
|
||||
"go.uber.org/atomic"
|
||||
)
|
||||
|
||||
const (
|
||||
@@ -125,17 +125,16 @@ func (m *cacheMeta) ToObjectInfo(bucket, object string) (o ObjectInfo) {
|
||||
|
||||
// represents disk cache struct
|
||||
type diskCache struct {
|
||||
dir string // caching directory
|
||||
quotaPct int // max usage in %
|
||||
// mark false if drive is offline
|
||||
online bool
|
||||
// mutex to protect updates to online variable
|
||||
onlineMutex *sync.RWMutex
|
||||
gcCounter uint64 // ref: https://golang.org/pkg/sync/atomic/#pkg-note-BUG
|
||||
// is set to 0 if drive is offline
|
||||
online uint32
|
||||
|
||||
dir string // caching directory
|
||||
quotaPct int // max usage in %
|
||||
pool sync.Pool
|
||||
after int // minimum accesses before an object is cached.
|
||||
lowWatermark int
|
||||
highWatermark int
|
||||
gcCounter atomic.Uint64
|
||||
// nsMutex namespace lock
|
||||
nsMutex *nsLockMap
|
||||
// Object functions pointing to the corresponding functions of backend implementation.
|
||||
@@ -153,8 +152,7 @@ func newDiskCache(dir string, quotaPct, after, lowWatermark, highWatermark int)
|
||||
after: after,
|
||||
lowWatermark: lowWatermark,
|
||||
highWatermark: highWatermark,
|
||||
online: true,
|
||||
onlineMutex: &sync.RWMutex{},
|
||||
online: 1,
|
||||
pool: sync.Pool{
|
||||
New: func() interface{} {
|
||||
b := disk.AlignedBlock(int(cacheBlkSize))
|
||||
@@ -340,27 +338,25 @@ func (c *diskCache) purge(ctx context.Context) {
|
||||
}
|
||||
|
||||
func (c *diskCache) incGCCounter() {
|
||||
c.gcCounter.Add(uint64(1))
|
||||
atomic.AddUint64(&c.gcCounter, 1)
|
||||
}
|
||||
|
||||
func (c *diskCache) resetGCCounter() {
|
||||
c.gcCounter.Store(uint64(0))
|
||||
atomic.StoreUint64(&c.gcCounter, 0)
|
||||
}
|
||||
|
||||
func (c *diskCache) gcCount() uint64 {
|
||||
return c.gcCounter.Load()
|
||||
return atomic.LoadUint64(&c.gcCounter)
|
||||
}
|
||||
|
||||
// sets cache drive status
|
||||
func (c *diskCache) setOnline(status bool) {
|
||||
c.onlineMutex.Lock()
|
||||
c.online = status
|
||||
c.onlineMutex.Unlock()
|
||||
func (c *diskCache) setOffline() {
|
||||
atomic.StoreUint32(&c.online, 0)
|
||||
}
|
||||
|
||||
// returns true if cache drive is online
|
||||
func (c *diskCache) IsOnline() bool {
|
||||
c.onlineMutex.RLock()
|
||||
defer c.onlineMutex.RUnlock()
|
||||
return c.online
|
||||
return atomic.LoadUint32(&c.online) != 0
|
||||
}
|
||||
|
||||
// Stat returns ObjectInfo from disk cache
|
||||
@@ -680,7 +676,8 @@ func (c *diskCache) Put(ctx context.Context, bucket, object string, data io.Read
|
||||
}
|
||||
n, err := c.bitrotWriteToCache(cachePath, cacheDataFile, reader, actualSize)
|
||||
if IsErr(err, baseErrs...) {
|
||||
c.setOnline(false)
|
||||
// take the cache drive offline
|
||||
c.setOffline()
|
||||
}
|
||||
if err != nil {
|
||||
removeAll(cachePath)
|
||||
@@ -727,7 +724,8 @@ func (c *diskCache) putRange(ctx context.Context, bucket, object string, data io
|
||||
cacheFile := MustGetUUID()
|
||||
n, err := c.bitrotWriteToCache(cachePath, cacheFile, reader, actualSize)
|
||||
if IsErr(err, baseErrs...) {
|
||||
c.setOnline(false)
|
||||
// take the cache drive offline
|
||||
c.setOffline()
|
||||
}
|
||||
if err != nil {
|
||||
removeAll(cachePath)
|
||||
|
||||
@@ -16,46 +16,44 @@
|
||||
|
||||
package cmd
|
||||
|
||||
import (
|
||||
"go.uber.org/atomic"
|
||||
)
|
||||
import "sync/atomic"
|
||||
|
||||
// CacheStats - represents bytes served from cache,
|
||||
// cache hits and cache misses.
|
||||
type CacheStats struct {
|
||||
BytesServed atomic.Uint64
|
||||
Hits atomic.Uint64
|
||||
Misses atomic.Uint64
|
||||
BytesServed uint64
|
||||
Hits uint64
|
||||
Misses uint64
|
||||
}
|
||||
|
||||
// Increase total bytes served from cache
|
||||
func (s *CacheStats) incBytesServed(n int64) {
|
||||
s.BytesServed.Add(uint64(n))
|
||||
atomic.AddUint64(&s.BytesServed, uint64(n))
|
||||
}
|
||||
|
||||
// Increase cache hit by 1
|
||||
func (s *CacheStats) incHit() {
|
||||
s.Hits.Add(uint64(1))
|
||||
atomic.AddUint64(&s.Hits, 1)
|
||||
}
|
||||
|
||||
// Increase cache miss by 1
|
||||
func (s *CacheStats) incMiss() {
|
||||
s.Misses.Add(uint64(1))
|
||||
atomic.AddUint64(&s.Misses, 1)
|
||||
}
|
||||
|
||||
// Get total bytes served
|
||||
func (s *CacheStats) getBytesServed() uint64 {
|
||||
return s.BytesServed.Load()
|
||||
return atomic.LoadUint64(&s.BytesServed)
|
||||
}
|
||||
|
||||
// Get total cache hits
|
||||
func (s *CacheStats) getHits() uint64 {
|
||||
return s.Hits.Load()
|
||||
return atomic.LoadUint64(&s.Hits)
|
||||
}
|
||||
|
||||
// Get total cache misses
|
||||
func (s *CacheStats) getMisses() uint64 {
|
||||
return s.Misses.Load()
|
||||
return atomic.LoadUint64(&s.Misses)
|
||||
}
|
||||
|
||||
// Prepare new CacheStats structure
|
||||
|
||||
@@ -78,7 +78,7 @@ func TestGetCachedLoc(t *testing.T) {
|
||||
// find cache drive where object would be hashed
|
||||
index := c.hashIndex(bucketName, objectName)
|
||||
// turn off drive by setting online status to false
|
||||
c.cache[index].online = false
|
||||
c.cache[index].setOffline()
|
||||
cfs, err := c.getCacheLoc(bucketName, objectName)
|
||||
if n == 1 && err == errDiskNotFound {
|
||||
continue
|
||||
@@ -118,7 +118,7 @@ func TestGetCacheMaxUse(t *testing.T) {
|
||||
// find cache drive where object would be hashed
|
||||
index := c.hashIndex(bucketName, objectName)
|
||||
// turn off drive by setting online status to false
|
||||
c.cache[index].online = false
|
||||
c.cache[index].setOffline()
|
||||
cb, err := c.getCacheLoc(bucketName, objectName)
|
||||
if n == 1 && err == errDiskNotFound {
|
||||
continue
|
||||
|
||||
@@ -18,57 +18,56 @@ package cmd
|
||||
|
||||
import (
|
||||
"net/http"
|
||||
|
||||
"go.uber.org/atomic"
|
||||
"sync/atomic"
|
||||
)
|
||||
|
||||
// RequestStats - counts for Get and Head requests
|
||||
type RequestStats struct {
|
||||
Get atomic.Uint64 `json:"Get"`
|
||||
Head atomic.Uint64 `json:"Head"`
|
||||
Put atomic.Uint64 `json:"Put"`
|
||||
Post atomic.Uint64 `json:"Post"`
|
||||
Get uint64 `json:"Get"`
|
||||
Head uint64 `json:"Head"`
|
||||
Put uint64 `json:"Put"`
|
||||
Post uint64 `json:"Post"`
|
||||
}
|
||||
|
||||
// Metrics - represents bytes served from backend
|
||||
// only implemented for S3 Gateway
|
||||
type Metrics struct {
|
||||
bytesReceived atomic.Uint64
|
||||
bytesSent atomic.Uint64
|
||||
bytesReceived uint64
|
||||
bytesSent uint64
|
||||
requestStats RequestStats
|
||||
}
|
||||
|
||||
// IncBytesReceived - Increase total bytes received from gateway backend
|
||||
func (s *Metrics) IncBytesReceived(n uint64) {
|
||||
s.bytesReceived.Add(n)
|
||||
atomic.AddUint64(&s.bytesReceived, n)
|
||||
}
|
||||
|
||||
// GetBytesReceived - Get total bytes received from gateway backend
|
||||
func (s *Metrics) GetBytesReceived() uint64 {
|
||||
return s.bytesReceived.Load()
|
||||
return atomic.LoadUint64(&s.bytesReceived)
|
||||
}
|
||||
|
||||
// IncBytesSent - Increase total bytes sent to gateway backend
|
||||
func (s *Metrics) IncBytesSent(n uint64) {
|
||||
s.bytesSent.Add(n)
|
||||
atomic.AddUint64(&s.bytesSent, n)
|
||||
}
|
||||
|
||||
// GetBytesSent - Get total bytes received from gateway backend
|
||||
func (s *Metrics) GetBytesSent() uint64 {
|
||||
return s.bytesSent.Load()
|
||||
return atomic.LoadUint64(&s.bytesSent)
|
||||
}
|
||||
|
||||
// IncRequests - Increase request count sent to gateway backend by 1
|
||||
func (s *Metrics) IncRequests(method string) {
|
||||
// Only increment for Head & Get requests, else no op
|
||||
if method == http.MethodGet {
|
||||
s.requestStats.Get.Add(1)
|
||||
atomic.AddUint64(&s.requestStats.Get, 1)
|
||||
} else if method == http.MethodHead {
|
||||
s.requestStats.Head.Add(1)
|
||||
atomic.AddUint64(&s.requestStats.Head, 1)
|
||||
} else if method == http.MethodPut {
|
||||
s.requestStats.Put.Add(1)
|
||||
atomic.AddUint64(&s.requestStats.Put, 1)
|
||||
} else if method == http.MethodPost {
|
||||
s.requestStats.Post.Add(1)
|
||||
atomic.AddUint64(&s.requestStats.Post, 1)
|
||||
}
|
||||
}
|
||||
|
||||
|
||||
@@ -18,7 +18,6 @@ package cmd
|
||||
|
||||
import (
|
||||
"context"
|
||||
"sync"
|
||||
"time"
|
||||
|
||||
"github.com/minio/minio/cmd/logger"
|
||||
@@ -38,7 +37,7 @@ var leaderLockTimeout = newDynamicTimeout(time.Minute, time.Minute)
|
||||
func newBgHealSequence(numDisks int) *healSequence {
|
||||
|
||||
reqInfo := &logger.ReqInfo{API: "BackgroundHeal"}
|
||||
ctx := logger.SetReqInfo(GlobalContext, reqInfo)
|
||||
ctx, cancelCtx := context.WithCancel(logger.SetReqInfo(GlobalContext, reqInfo))
|
||||
|
||||
hs := madmin.HealOpts{
|
||||
// Remove objects that do not have read-quorum
|
||||
@@ -56,15 +55,13 @@ func newBgHealSequence(numDisks int) *healSequence {
|
||||
Summary: healNotStartedStatus,
|
||||
HealSettings: hs,
|
||||
NumDisks: numDisks,
|
||||
updateLock: &sync.RWMutex{},
|
||||
},
|
||||
traverseAndHealDoneCh: make(chan error),
|
||||
stopSignalCh: make(chan struct{}),
|
||||
ctx: ctx,
|
||||
reportProgress: false,
|
||||
scannedItemsMap: make(map[madmin.HealItemType]int64),
|
||||
healedItemsMap: make(map[madmin.HealItemType]int64),
|
||||
healFailedItemsMap: make(map[string]int64),
|
||||
cancelCtx: cancelCtx,
|
||||
ctx: ctx,
|
||||
reportProgress: false,
|
||||
scannedItemsMap: make(map[madmin.HealItemType]int64),
|
||||
healedItemsMap: make(map[madmin.HealItemType]int64),
|
||||
healFailedItemsMap: make(map[string]int64),
|
||||
}
|
||||
}
|
||||
|
||||
@@ -96,7 +93,12 @@ func healErasureSet(ctx context.Context, setIndex int, xlObj *xlObjects) error {
|
||||
if ok {
|
||||
break
|
||||
}
|
||||
time.Sleep(time.Second)
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
return nil
|
||||
case <-time.After(time.Second):
|
||||
continue
|
||||
}
|
||||
}
|
||||
|
||||
// Heal all buckets with all objects
|
||||
@@ -155,7 +157,12 @@ func execLeaderTasks(ctx context.Context, z *xlZones) {
|
||||
if ok {
|
||||
break
|
||||
}
|
||||
time.Sleep(time.Second)
|
||||
select {
|
||||
case <-ctx.Done():
|
||||
return
|
||||
case <-time.After(time.Second):
|
||||
continue
|
||||
}
|
||||
}
|
||||
for {
|
||||
select {
|
||||
|
||||
@@ -21,61 +21,61 @@ import (
|
||||
"net/http"
|
||||
"strings"
|
||||
"sync"
|
||||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"github.com/minio/minio/cmd/logger"
|
||||
"github.com/prometheus/client_golang/prometheus"
|
||||
"go.uber.org/atomic"
|
||||
)
|
||||
|
||||
// ConnStats - Network statistics
|
||||
// Count total input/output transferred bytes during
|
||||
// the server's life.
|
||||
type ConnStats struct {
|
||||
totalInputBytes atomic.Uint64
|
||||
totalOutputBytes atomic.Uint64
|
||||
s3InputBytes atomic.Uint64
|
||||
s3OutputBytes atomic.Uint64
|
||||
totalInputBytes uint64
|
||||
totalOutputBytes uint64
|
||||
s3InputBytes uint64
|
||||
s3OutputBytes uint64
|
||||
}
|
||||
|
||||
// Increase total input bytes
|
||||
func (s *ConnStats) incInputBytes(n int) {
|
||||
s.totalInputBytes.Add(uint64(n))
|
||||
atomic.AddUint64(&s.totalInputBytes, uint64(n))
|
||||
}
|
||||
|
||||
// Increase total output bytes
|
||||
func (s *ConnStats) incOutputBytes(n int) {
|
||||
s.totalOutputBytes.Add(uint64(n))
|
||||
atomic.AddUint64(&s.totalOutputBytes, uint64(n))
|
||||
}
|
||||
|
||||
// Return total input bytes
|
||||
func (s *ConnStats) getTotalInputBytes() uint64 {
|
||||
return s.totalInputBytes.Load()
|
||||
return atomic.LoadUint64(&s.totalInputBytes)
|
||||
}
|
||||
|
||||
// Return total output bytes
|
||||
func (s *ConnStats) getTotalOutputBytes() uint64 {
|
||||
return s.totalOutputBytes.Load()
|
||||
return atomic.LoadUint64(&s.totalOutputBytes)
|
||||
}
|
||||
|
||||
// Increase outbound input bytes
|
||||
func (s *ConnStats) incS3InputBytes(n int) {
|
||||
s.s3InputBytes.Add(uint64(n))
|
||||
atomic.AddUint64(&s.s3InputBytes, uint64(n))
|
||||
}
|
||||
|
||||
// Increase outbound output bytes
|
||||
func (s *ConnStats) incS3OutputBytes(n int) {
|
||||
s.s3OutputBytes.Add(uint64(n))
|
||||
atomic.AddUint64(&s.s3OutputBytes, uint64(n))
|
||||
}
|
||||
|
||||
// Return outbound input bytes
|
||||
func (s *ConnStats) getS3InputBytes() uint64 {
|
||||
return s.s3InputBytes.Load()
|
||||
return atomic.LoadUint64(&s.s3InputBytes)
|
||||
}
|
||||
|
||||
// Return outbound output bytes
|
||||
func (s *ConnStats) getS3OutputBytes() uint64 {
|
||||
return s.s3OutputBytes.Load()
|
||||
return atomic.LoadUint64(&s.s3OutputBytes)
|
||||
}
|
||||
|
||||
// Return connection stats (total input/output bytes and total s3 input/output bytes)
|
||||
@@ -178,7 +178,7 @@ func (st *HTTPStats) updateStats(api string, r *http.Request, w *logger.Response
|
||||
}
|
||||
}
|
||||
|
||||
if r.Method == "GET" {
|
||||
if r.Method == http.MethodGet {
|
||||
// Increment the prometheus http request response histogram with appropriate label
|
||||
httpRequestsDuration.With(prometheus.Labels{"api": api}).Observe(durationSecs)
|
||||
}
|
||||
|
||||
@@ -39,9 +39,9 @@ type mergeWalk struct {
|
||||
// mergeWalkPool's purpose is to maintain active mergeWalk go-routines in a map so that
|
||||
// it can be looked up across related list calls.
|
||||
type MergeWalkPool struct {
|
||||
sync.Mutex
|
||||
pool map[listParams][]mergeWalk
|
||||
timeOut time.Duration
|
||||
lock *sync.Mutex
|
||||
}
|
||||
|
||||
// NewMergeWalkPool - initialize new tree walk pool.
|
||||
@@ -49,7 +49,6 @@ func NewMergeWalkPool(timeout time.Duration) *MergeWalkPool {
|
||||
tPool := &MergeWalkPool{
|
||||
pool: make(map[listParams][]mergeWalk),
|
||||
timeOut: timeout,
|
||||
lock: &sync.Mutex{},
|
||||
}
|
||||
return tPool
|
||||
}
|
||||
@@ -58,9 +57,9 @@ func NewMergeWalkPool(timeout time.Duration) *MergeWalkPool {
|
||||
// listParams, removes it from the pool, and returns the MergeWalkResult
|
||||
// channel.
|
||||
// Returns nil if listParams does not have an asccociated mergeWalk.
|
||||
func (t MergeWalkPool) Release(params listParams) ([]FileInfoCh, chan struct{}) {
|
||||
t.lock.Lock()
|
||||
defer t.lock.Unlock()
|
||||
func (t *MergeWalkPool) Release(params listParams) ([]FileInfoCh, chan struct{}) {
|
||||
t.Lock()
|
||||
defer t.Unlock()
|
||||
walks, ok := t.pool[params] // Pick the valid walks.
|
||||
if ok {
|
||||
if len(walks) > 0 {
|
||||
@@ -88,9 +87,9 @@ func (t MergeWalkPool) Release(params listParams) ([]FileInfoCh, chan struct{})
|
||||
// 2) Relase() signals the timer go-routine to end on endTimerCh.
|
||||
// During listing the timer should not timeout and end the mergeWalk go-routine, hence the
|
||||
// timer go-routine should be ended.
|
||||
func (t MergeWalkPool) Set(params listParams, resultChs []FileInfoCh, endWalkCh chan struct{}) {
|
||||
t.lock.Lock()
|
||||
defer t.lock.Unlock()
|
||||
func (t *MergeWalkPool) Set(params listParams, resultChs []FileInfoCh, endWalkCh chan struct{}) {
|
||||
t.Lock()
|
||||
defer t.Unlock()
|
||||
|
||||
// Should be a buffered channel so that Release() never blocks.
|
||||
endTimerCh := make(chan struct{}, 1)
|
||||
@@ -111,7 +110,7 @@ func (t MergeWalkPool) Set(params listParams, resultChs []FileInfoCh, endWalkCh
|
||||
case <-time.After(t.timeOut):
|
||||
// Timeout has expired. Remove the mergeWalk from mergeWalkPool and
|
||||
// end the mergeWalk go-routine.
|
||||
t.lock.Lock()
|
||||
t.Lock()
|
||||
walks, ok := t.pool[params]
|
||||
if ok {
|
||||
// Trick of filtering without allocating
|
||||
@@ -135,7 +134,7 @@ func (t MergeWalkPool) Set(params listParams, resultChs []FileInfoCh, endWalkCh
|
||||
}
|
||||
// Signal the mergeWalk go-routine to die.
|
||||
close(endWalkCh)
|
||||
t.lock.Unlock()
|
||||
t.Unlock()
|
||||
case <-endTimerCh:
|
||||
return
|
||||
}
|
||||
|
||||
@@ -80,22 +80,22 @@ func TestManyMergeWalksSameParam(t *testing.T) {
|
||||
tw.Set(params, walkChs, endWalkCh)
|
||||
}
|
||||
|
||||
tw.lock.Lock()
|
||||
tw.Lock()
|
||||
if walks, ok := tw.pool[params]; ok {
|
||||
if len(walks) != 10 {
|
||||
t.Error("There aren't as many walks as were Set")
|
||||
}
|
||||
}
|
||||
tw.lock.Unlock()
|
||||
tw.Unlock()
|
||||
for i := 0; i < 10; i++ {
|
||||
tw.lock.Lock()
|
||||
tw.Lock()
|
||||
if walks, ok := tw.pool[params]; ok {
|
||||
// Before ith Release we should have 10-i treeWalk go-routines.
|
||||
if 10-i != len(walks) {
|
||||
t.Error("There aren't as many walks as were Set")
|
||||
}
|
||||
}
|
||||
tw.lock.Unlock()
|
||||
tw.Unlock()
|
||||
tw.Release(params)
|
||||
}
|
||||
}
|
||||
|
||||
@@ -19,6 +19,7 @@ package cmd
|
||||
import (
|
||||
"net/http"
|
||||
"strings"
|
||||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"github.com/minio/minio/cmd/logger"
|
||||
@@ -190,7 +191,7 @@ func gatewayMetricsPrometheus(ch chan<- prometheus.Metric) {
|
||||
"Total number of requests made to "+globalGatewayName+" by current MinIO Gateway",
|
||||
[]string{"method"}, nil),
|
||||
prometheus.CounterValue,
|
||||
float64(s.Get.Load()),
|
||||
float64(atomic.LoadUint64(&s.Get)),
|
||||
http.MethodGet,
|
||||
)
|
||||
ch <- prometheus.MustNewConstMetric(
|
||||
@@ -199,7 +200,7 @@ func gatewayMetricsPrometheus(ch chan<- prometheus.Metric) {
|
||||
"Total number of requests made to "+globalGatewayName+" by current MinIO Gateway",
|
||||
[]string{"method"}, nil),
|
||||
prometheus.CounterValue,
|
||||
float64(s.Head.Load()),
|
||||
float64(atomic.LoadUint64(&s.Head)),
|
||||
http.MethodHead,
|
||||
)
|
||||
ch <- prometheus.MustNewConstMetric(
|
||||
@@ -208,7 +209,7 @@ func gatewayMetricsPrometheus(ch chan<- prometheus.Metric) {
|
||||
"Total number of requests made to "+globalGatewayName+" by current MinIO Gateway",
|
||||
[]string{"method"}, nil),
|
||||
prometheus.CounterValue,
|
||||
float64(s.Put.Load()),
|
||||
float64(atomic.LoadUint64(&s.Put)),
|
||||
http.MethodPut,
|
||||
)
|
||||
ch <- prometheus.MustNewConstMetric(
|
||||
@@ -217,7 +218,7 @@ func gatewayMetricsPrometheus(ch chan<- prometheus.Metric) {
|
||||
"Total number of requests made to "+globalGatewayName+" by current MinIO Gateway",
|
||||
[]string{"method"}, nil),
|
||||
prometheus.CounterValue,
|
||||
float64(s.Post.Load()),
|
||||
float64(atomic.LoadUint64(&s.Post)),
|
||||
http.MethodPost,
|
||||
)
|
||||
}
|
||||
|
||||
@@ -42,7 +42,7 @@ const (
|
||||
)
|
||||
|
||||
// Global object layer mutex, used for safely updating object layer.
|
||||
var globalObjLayerMutex *sync.RWMutex
|
||||
var globalObjLayerMutex sync.RWMutex
|
||||
|
||||
// Global object layer, only accessed by globalObjectAPI.
|
||||
var globalObjectAPI ObjectLayer
|
||||
@@ -50,11 +50,6 @@ var globalObjectAPI ObjectLayer
|
||||
//Global cacheObjects, only accessed by newCacheObjectsFn().
|
||||
var globalCacheObjectAPI CacheObjectLayer
|
||||
|
||||
func init() {
|
||||
// Initialize this once per server initialization.
|
||||
globalObjLayerMutex = &sync.RWMutex{}
|
||||
}
|
||||
|
||||
// Checks if the object is a directory, this logic uses
|
||||
// if size == 0 and object ends with SlashSeparator then
|
||||
// returns true.
|
||||
|
||||
@@ -55,9 +55,9 @@ type treeWalk struct {
|
||||
// treeWalkPool's purpose is to maintain active treeWalk go-routines in a map so that
|
||||
// it can be looked up across related list calls.
|
||||
type TreeWalkPool struct {
|
||||
sync.Mutex
|
||||
pool map[listParams][]treeWalk
|
||||
timeOut time.Duration
|
||||
lock *sync.Mutex
|
||||
}
|
||||
|
||||
// NewTreeWalkPool - initialize new tree walk pool.
|
||||
@@ -65,7 +65,6 @@ func NewTreeWalkPool(timeout time.Duration) *TreeWalkPool {
|
||||
tPool := &TreeWalkPool{
|
||||
pool: make(map[listParams][]treeWalk),
|
||||
timeOut: timeout,
|
||||
lock: &sync.Mutex{},
|
||||
}
|
||||
return tPool
|
||||
}
|
||||
@@ -74,9 +73,9 @@ func NewTreeWalkPool(timeout time.Duration) *TreeWalkPool {
|
||||
// listParams, removes it from the pool, and returns the TreeWalkResult
|
||||
// channel.
|
||||
// Returns nil if listParams does not have an asccociated treeWalk.
|
||||
func (t TreeWalkPool) Release(params listParams) (resultCh chan TreeWalkResult, endWalkCh chan struct{}) {
|
||||
t.lock.Lock()
|
||||
defer t.lock.Unlock()
|
||||
func (t *TreeWalkPool) Release(params listParams) (resultCh chan TreeWalkResult, endWalkCh chan struct{}) {
|
||||
t.Lock()
|
||||
defer t.Unlock()
|
||||
walks, ok := t.pool[params] // Pick the valid walks.
|
||||
if ok {
|
||||
if len(walks) > 0 {
|
||||
@@ -104,9 +103,9 @@ func (t TreeWalkPool) Release(params listParams) (resultCh chan TreeWalkResult,
|
||||
// 2) Relase() signals the timer go-routine to end on endTimerCh.
|
||||
// During listing the timer should not timeout and end the treeWalk go-routine, hence the
|
||||
// timer go-routine should be ended.
|
||||
func (t TreeWalkPool) Set(params listParams, resultCh chan TreeWalkResult, endWalkCh chan struct{}) {
|
||||
t.lock.Lock()
|
||||
defer t.lock.Unlock()
|
||||
func (t *TreeWalkPool) Set(params listParams, resultCh chan TreeWalkResult, endWalkCh chan struct{}) {
|
||||
t.Lock()
|
||||
defer t.Unlock()
|
||||
|
||||
// Should be a buffered channel so that Release() never blocks.
|
||||
endTimerCh := make(chan struct{}, 1)
|
||||
@@ -125,7 +124,7 @@ func (t TreeWalkPool) Set(params listParams, resultCh chan TreeWalkResult, endWa
|
||||
case <-time.After(t.timeOut):
|
||||
// Timeout has expired. Remove the treeWalk from treeWalkPool and
|
||||
// end the treeWalk go-routine.
|
||||
t.lock.Lock()
|
||||
t.Lock()
|
||||
walks, ok := t.pool[params]
|
||||
if ok {
|
||||
// Trick of filtering without allocating
|
||||
@@ -149,7 +148,7 @@ func (t TreeWalkPool) Set(params listParams, resultCh chan TreeWalkResult, endWa
|
||||
}
|
||||
// Signal the treeWalk go-routine to die.
|
||||
close(endWalkCh)
|
||||
t.lock.Unlock()
|
||||
t.Unlock()
|
||||
case <-endTimerCh:
|
||||
return
|
||||
}
|
||||
|
||||
@@ -80,22 +80,22 @@ func TestManyWalksSameParam(t *testing.T) {
|
||||
tw.Set(params, resultCh, endWalkCh)
|
||||
}
|
||||
|
||||
tw.lock.Lock()
|
||||
tw.Lock()
|
||||
if walks, ok := tw.pool[params]; ok {
|
||||
if len(walks) != 10 {
|
||||
t.Error("There aren't as many walks as were Set")
|
||||
}
|
||||
}
|
||||
tw.lock.Unlock()
|
||||
tw.Unlock()
|
||||
for i := 0; i < 10; i++ {
|
||||
tw.lock.Lock()
|
||||
tw.Lock()
|
||||
if walks, ok := tw.pool[params]; ok {
|
||||
// Before ith Release we should have 10-i treeWalk go-routines.
|
||||
if 10-i != len(walks) {
|
||||
t.Error("There aren't as many walks as were Set")
|
||||
}
|
||||
}
|
||||
tw.lock.Unlock()
|
||||
tw.Unlock()
|
||||
tw.Release(params)
|
||||
}
|
||||
}
|
||||
|
||||
Reference in New Issue
Block a user