| 
									
										
										
										
											2016-03-28 12:52:38 +08:00
										 |  |  | /* | 
					
						
							| 
									
										
										
										
											2019-04-10 02:39:42 +08:00
										 |  |  |  * MinIO Cloud Storage, (C) 2016 MinIO, Inc. | 
					
						
							| 
									
										
										
										
											2016-03-28 12:52:38 +08:00
										 |  |  |  * | 
					
						
							|  |  |  |  * 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. | 
					
						
							|  |  |  |  */ | 
					
						
							|  |  |  | 
 | 
					
						
							| 
									
										
										
										
											2016-08-19 07:23:42 +08:00
										 |  |  | package cmd | 
					
						
							| 
									
										
										
										
											2016-03-28 12:52:38 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2016-06-01 11:23:31 +08:00
										 |  |  | import ( | 
					
						
							| 
									
										
										
										
											2018-04-06 06:04:40 +08:00
										 |  |  | 	"context" | 
					
						
							| 
									
										
										
										
											2016-06-20 04:35:26 +08:00
										 |  |  | 	"io" | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 	"sync" | 
					
						
							| 
									
										
										
										
											2017-11-26 03:58:29 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-04-06 06:04:40 +08:00
										 |  |  | 	"github.com/minio/minio/cmd/logger" | 
					
						
							| 
									
										
										
										
											2016-06-01 11:23:31 +08:00
										 |  |  | ) | 
					
						
							| 
									
										
										
										
											2016-03-28 12:52:38 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | // Reads in parallel from readers.
 | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | type parallelReader struct { | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | 	readers       []io.ReaderAt | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	dataBlocks    int | 
					
						
							|  |  |  | 	offset        int64 | 
					
						
							|  |  |  | 	shardSize     int64 | 
					
						
							|  |  |  | 	shardFileSize int64 | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | 	buf           [][]byte | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | } | 
					
						
							|  |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | // newParallelReader returns parallelReader.
 | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | func newParallelReader(readers []io.ReaderAt, e Erasure, offset, totalLength int64) *parallelReader { | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	return ¶llelReader{ | 
					
						
							|  |  |  | 		readers, | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | 		e.dataBlocks, | 
					
						
							|  |  |  | 		(offset / e.blockSize) * e.ShardSize(), | 
					
						
							|  |  |  | 		e.ShardSize(), | 
					
						
							|  |  |  | 		e.ShardFileSize(totalLength), | 
					
						
							|  |  |  | 		make([][]byte, len(readers)), | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 	} | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | } | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | // Returns if buf can be erasure decoded.
 | 
					
						
							|  |  |  | func (p *parallelReader) canDecode(buf [][]byte) bool { | 
					
						
							|  |  |  | 	bufCount := 0 | 
					
						
							|  |  |  | 	for _, b := range buf { | 
					
						
							|  |  |  | 		if b != nil { | 
					
						
							|  |  |  | 			bufCount++ | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 		} | 
					
						
							|  |  |  | 	} | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	return bufCount >= p.dataBlocks | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | } | 
					
						
							|  |  |  | 
 | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | // Read reads from readers in parallel. Returns p.dataBlocks number of bufs.
 | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | func (p *parallelReader) Read() ([][]byte, error) { | 
					
						
							|  |  |  | 	newBuf := make([][]byte, len(p.readers)) | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 	var newBufLK sync.RWMutex | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	if p.offset+p.shardSize > p.shardFileSize { | 
					
						
							|  |  |  | 		p.shardSize = p.shardFileSize - p.offset | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 	} | 
					
						
							|  |  |  | 
 | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 	readTriggerCh := make(chan bool, len(p.readers)) | 
					
						
							|  |  |  | 	for i := 0; i < p.dataBlocks; i++ { | 
					
						
							|  |  |  | 		// Setup read triggers for p.dataBlocks number of reads so that it reads in parallel.
 | 
					
						
							|  |  |  | 		readTriggerCh <- true | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 	} | 
					
						
							|  |  |  | 
 | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 	readerIndex := 0 | 
					
						
							|  |  |  | 	var wg sync.WaitGroup | 
					
						
							|  |  |  | 	// if readTrigger is true, it implies next disk.ReadAt() should be tried
 | 
					
						
							|  |  |  | 	// if readTrigger is false, it implies previous disk.ReadAt() was successful and there is no need
 | 
					
						
							|  |  |  | 	// to try reading the next disk.
 | 
					
						
							|  |  |  | 	for readTrigger := range readTriggerCh { | 
					
						
							|  |  |  | 		newBufLK.RLock() | 
					
						
							|  |  |  | 		canDecode := p.canDecode(newBuf) | 
					
						
							|  |  |  | 		newBufLK.RUnlock() | 
					
						
							|  |  |  | 		if canDecode { | 
					
						
							|  |  |  | 			break | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 		} | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 		if readerIndex == len(p.readers) { | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 			break | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 		} | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 		if !readTrigger { | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 			continue | 
					
						
							| 
									
										
										
										
											2016-06-25 09:00:34 +08:00
										 |  |  | 		} | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 		wg.Add(1) | 
					
						
							|  |  |  | 		go func(i int) { | 
					
						
							|  |  |  | 			defer wg.Done() | 
					
						
							|  |  |  | 			disk := p.readers[i] | 
					
						
							|  |  |  | 			if disk == nil { | 
					
						
							|  |  |  | 				// Since disk is nil, trigger another read.
 | 
					
						
							|  |  |  | 				readTriggerCh <- true | 
					
						
							|  |  |  | 				return | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 			} | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 			if p.buf[i] == nil { | 
					
						
							|  |  |  | 				// Reading first time on this disk, hence the buffer needs to be allocated.
 | 
					
						
							|  |  |  | 				// Subsequent reads will re-use this buffer.
 | 
					
						
							|  |  |  | 				p.buf[i] = make([]byte, p.shardSize) | 
					
						
							|  |  |  | 			} | 
					
						
							|  |  |  | 			// For the last shard, the shardsize might be less than previous shard sizes.
 | 
					
						
							|  |  |  | 			// Hence the following statement ensures that the buffer size is reset to the right size.
 | 
					
						
							|  |  |  | 			p.buf[i] = p.buf[i][:p.shardSize] | 
					
						
							|  |  |  | 			_, err := disk.ReadAt(p.buf[i], p.offset) | 
					
						
							|  |  |  | 			if err != nil { | 
					
						
							|  |  |  | 				p.readers[i] = nil | 
					
						
							|  |  |  | 				// Since ReadAt returned error, trigger another read.
 | 
					
						
							|  |  |  | 				readTriggerCh <- true | 
					
						
							|  |  |  | 				return | 
					
						
							|  |  |  | 			} | 
					
						
							|  |  |  | 			newBufLK.Lock() | 
					
						
							|  |  |  | 			newBuf[i] = p.buf[i] | 
					
						
							|  |  |  | 			newBufLK.Unlock() | 
					
						
							|  |  |  | 			// Since ReadAt returned success, there is no need to trigger another read.
 | 
					
						
							|  |  |  | 			readTriggerCh <- false | 
					
						
							|  |  |  | 		}(readerIndex) | 
					
						
							|  |  |  | 		readerIndex++ | 
					
						
							|  |  |  | 	} | 
					
						
							|  |  |  | 	wg.Wait() | 
					
						
							| 
									
										
										
										
											2017-08-15 09:08:42 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2019-01-18 23:48:25 +08:00
										 |  |  | 	if p.canDecode(newBuf) { | 
					
						
							|  |  |  | 		p.offset += p.shardSize | 
					
						
							|  |  |  | 		return newBuf, nil | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	} | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	return nil, errXLReadQuorum | 
					
						
							|  |  |  | } | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | // Decode reads from readers, reconstructs data if needed and writes the data to the writer.
 | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | func (e Erasure) Decode(ctx context.Context, writer io.Writer, readers []io.ReaderAt, offset, length, totalLength int64) error { | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	if offset < 0 || length < 0 { | 
					
						
							|  |  |  | 		logger.LogIf(ctx, errInvalidArgument) | 
					
						
							|  |  |  | 		return errInvalidArgument | 
					
						
							|  |  |  | 	} | 
					
						
							|  |  |  | 	if offset+length > totalLength { | 
					
						
							|  |  |  | 		logger.LogIf(ctx, errInvalidArgument) | 
					
						
							|  |  |  | 		return errInvalidArgument | 
					
						
							|  |  |  | 	} | 
					
						
							|  |  |  | 	if length == 0 { | 
					
						
							|  |  |  | 		return nil | 
					
						
							|  |  |  | 	} | 
					
						
							| 
									
										
										
										
											2018-03-05 06:16:45 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2019-01-17 20:58:18 +08:00
										 |  |  | 	reader := newParallelReader(readers, e, offset, totalLength) | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 
 | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 	startBlock := offset / e.blockSize | 
					
						
							|  |  |  | 	endBlock := (offset + length) / e.blockSize | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 
 | 
					
						
							|  |  |  | 	var bytesWritten int64 | 
					
						
							|  |  |  | 	for block := startBlock; block <= endBlock; block++ { | 
					
						
							|  |  |  | 		var blockOffset, blockLength int64 | 
					
						
							|  |  |  | 		switch { | 
					
						
							|  |  |  | 		case startBlock == endBlock: | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 			blockOffset = offset % e.blockSize | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 			blockLength = length | 
					
						
							|  |  |  | 		case block == startBlock: | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 			blockOffset = offset % e.blockSize | 
					
						
							|  |  |  | 			blockLength = e.blockSize - blockOffset | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 		case block == endBlock: | 
					
						
							|  |  |  | 			blockOffset = 0 | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 			blockLength = (offset + length) % e.blockSize | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 		default: | 
					
						
							|  |  |  | 			blockOffset = 0 | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 			blockLength = e.blockSize | 
					
						
							| 
									
										
										
										
											2016-06-25 09:00:34 +08:00
										 |  |  | 		} | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 		if blockLength == 0 { | 
					
						
							|  |  |  | 			break | 
					
						
							|  |  |  | 		} | 
					
						
							|  |  |  | 		bufs, err := reader.Read() | 
					
						
							| 
									
										
										
										
											2017-08-15 09:08:42 +08:00
										 |  |  | 		if err != nil { | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 			return err | 
					
						
							| 
									
										
										
										
											2017-08-15 09:08:42 +08:00
										 |  |  | 		} | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 		if err = e.DecodeDataBlocks(bufs); err != nil { | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 			logger.LogIf(ctx, err) | 
					
						
							|  |  |  | 			return err | 
					
						
							| 
									
										
										
										
											2016-06-28 04:24:55 +08:00
										 |  |  | 		} | 
					
						
							| 
									
										
										
										
											2018-08-24 14:35:37 +08:00
										 |  |  | 		n, err := writeDataBlocks(ctx, writer, bufs, e.dataBlocks, blockOffset, blockLength) | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 		if err != nil { | 
					
						
							|  |  |  | 			return err | 
					
						
							|  |  |  | 		} | 
					
						
							|  |  |  | 		bytesWritten += n | 
					
						
							|  |  |  | 	} | 
					
						
							|  |  |  | 	if bytesWritten != length { | 
					
						
							|  |  |  | 		logger.LogIf(ctx, errLessData) | 
					
						
							|  |  |  | 		return errLessData | 
					
						
							| 
									
										
										
										
											2016-06-28 04:24:55 +08:00
										 |  |  | 	} | 
					
						
							| 
									
										
										
										
											2018-08-07 06:14:08 +08:00
										 |  |  | 	return nil | 
					
						
							| 
									
										
										
										
											2016-06-28 04:24:55 +08:00
										 |  |  | } |