forked from m3db/m3
-
Notifications
You must be signed in to change notification settings - Fork 0
/
storage.go
447 lines (383 loc) · 10.8 KB
/
storage.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
// Copyright (c) 2018 Uber Technologies, Inc.
//
// Permission is hereby granted, free of charge, to any person obtaining a copy
// of this software and associated documentation files (the "Software"), to deal
// in the Software without restriction, including without limitation the rights
// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell
// copies of the Software, and to permit persons to whom the Software is
// furnished to do so, subject to the following conditions:
//
// The above copyright notice and this permission notice shall be included in
// all copies or substantial portions of the Software.
//
// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR
// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY,
// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE
// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER
// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM,
// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN
// THE SOFTWARE.
package local
import (
"context"
goerrors "errors"
"fmt"
"sync"
"time"
"github.com/m3db/m3/src/dbnode/storage/index"
"github.com/m3db/m3/src/query/block"
"github.com/m3db/m3/src/query/errors"
"github.com/m3db/m3/src/query/models"
"github.com/m3db/m3/src/query/storage"
"github.com/m3db/m3/src/query/util/execution"
xerrors "github.com/m3db/m3x/errors"
"github.com/m3db/m3x/ident"
"github.com/m3db/m3x/pool"
xtime "github.com/m3db/m3x/time"
)
var (
errNoLocalClustersFulfillsQuery = goerrors.New("no clusters can fulfill query")
)
type localStorage struct {
clusters Clusters
workerPool pool.ObjectPool
}
// NewStorage creates a new local Storage instance.
func NewStorage(clusters Clusters, workerPool pool.ObjectPool) storage.Storage {
return &localStorage{clusters: clusters, workerPool: workerPool}
}
func (s *localStorage) Fetch(ctx context.Context, query *storage.FetchQuery, options *storage.FetchOptions) (*storage.FetchResult, error) {
// Check if the query was interrupted.
select {
case <-ctx.Done():
return nil, ctx.Err()
case <-options.KillChan:
return nil, errors.ErrQueryInterrupted
default:
}
m3query, err := storage.FetchQueryToM3Query(query)
if err != nil {
return nil, err
}
// NB(r): Since we don't use a single index we fan out to each
// cluster that can completely fulfill this range and then prefer the
// highest resolution (most fine grained) results.
// This needs to be optimized, however this is a start.
var (
opts = storage.FetchOptionsToM3Options(options, query)
namespaces = s.clusters.ClusterNamespaces()
now = time.Now()
fetches = 0
result multiFetchResult
wg sync.WaitGroup
)
for _, namespace := range namespaces {
namespace := namespace // Capture var
clusterStart := now.Add(-1 * namespace.Attributes().Retention)
// Only include if cluster can completely fulfill the range
if clusterStart.After(query.Start) {
continue
}
fetches++
wg.Add(1)
go func() {
r, err := s.fetch(namespace, m3query, opts)
result.add(namespace.Attributes(), r, err)
wg.Done()
}()
}
if fetches == 0 {
return nil, errNoLocalClustersFulfillsQuery
}
wg.Wait()
if err := result.err.FinalError(); err != nil {
return nil, err
}
return result.result, nil
}
func (s *localStorage) fetch(
namespace ClusterNamespace,
query index.Query,
opts index.QueryOptions,
) (*storage.FetchResult, error) {
namespaceID := namespace.NamespaceID()
session := namespace.Session()
// TODO (nikunj): Handle second return param
iters, _, err := session.FetchTagged(namespaceID, query, opts)
if err != nil {
return nil, err
}
return storage.SeriesIteratorsToFetchResult(iters, namespaceID, s.workerPool)
}
func (s *localStorage) FetchTags(ctx context.Context, query *storage.FetchQuery, options *storage.FetchOptions) (*storage.SearchResults, error) {
// Check if the query was interrupted.
select {
case <-ctx.Done():
return nil, ctx.Err()
case <-options.KillChan:
return nil, errors.ErrQueryInterrupted
default:
}
m3query, err := storage.FetchQueryToM3Query(query)
if err != nil {
return nil, err
}
var (
opts = storage.FetchOptionsToM3Options(options, query)
namespaces = s.clusters.ClusterNamespaces()
now = time.Now()
fetches = 0
result multiFetchTagsResult
wg sync.WaitGroup
)
for _, namespace := range namespaces {
namespace := namespace // Capture var
clusterStart := now.Add(-1 * namespace.Attributes().Retention)
// Only include if cluster can completely fulfill the range
if clusterStart.After(query.Start) {
continue
}
fetches++
wg.Add(1)
go func() {
result.add(s.fetchTags(namespace, m3query, opts))
wg.Done()
}()
}
if fetches == 0 {
return nil, errNoLocalClustersFulfillsQuery
}
wg.Wait()
if err := result.err.FinalError(); err != nil {
return nil, err
}
return result.result, nil
}
func (s *localStorage) fetchTags(
namespace ClusterNamespace,
query index.Query,
opts index.QueryOptions,
) (*storage.SearchResults, error) {
namespaceID := namespace.NamespaceID()
session := namespace.Session()
// TODO (juchan): Handle second return param
iter, _, err := session.FetchTaggedIDs(namespaceID, query, opts)
if err != nil {
return nil, err
}
var metrics models.Metrics
for iter.Next() {
m, err := storage.FromM3IdentToMetric(iter.Current())
if err != nil {
return nil, err
}
metrics = append(metrics, m)
}
if err := iter.Err(); err != nil {
return nil, err
}
iter.Finalize()
return &storage.SearchResults{
Metrics: metrics,
}, nil
}
func (s *localStorage) Write(ctx context.Context, query *storage.WriteQuery) error {
// Check if the query was interrupted.
select {
case <-ctx.Done():
return ctx.Err()
default:
}
if query == nil {
return errors.ErrNilWriteQuery
}
id := query.Tags.ID()
common := &writeRequestCommon{
store: s,
annotation: query.Annotation,
unit: query.Unit,
id: id,
tagIterator: storage.TagsToIdentTagIterator(query.Tags),
attributes: query.Attributes,
}
requests := make([]execution.Request, len(query.Datapoints))
for idx, datapoint := range query.Datapoints {
requests[idx] = newWriteRequest(common, datapoint.Timestamp, datapoint.Value)
}
return execution.ExecuteParallel(ctx, requests)
}
func (s *localStorage) Type() storage.Type {
return storage.TypeLocalDC
}
func (s *localStorage) FetchBlocks(
ctx context.Context,
query *storage.FetchQuery,
options *storage.FetchOptions) (block.Result, error) {
fetchResult, err := s.Fetch(ctx, query, options)
if err != nil {
return block.Result{}, err
}
res, err := storage.FetchResultToBlockResult(fetchResult, query)
if err != nil {
return block.Result{}, err
}
return res, nil
}
func (s *localStorage) Close() error {
return nil
}
func (w *writeRequest) Process(ctx context.Context) error {
common := w.writeRequestCommon
store := common.store
id := ident.StringID(common.id)
var (
namespace ClusterNamespace
err error
)
switch common.attributes.MetricsType {
case storage.UnaggregatedMetricsType:
namespace = store.clusters.UnaggregatedClusterNamespace()
case storage.AggregatedMetricsType:
attrs := RetentionResolution{
Retention: common.attributes.Retention,
Resolution: common.attributes.Resolution,
}
var exists bool
namespace, exists = store.clusters.AggregatedClusterNamespace(attrs)
if !exists {
err = fmt.Errorf("no configured cluster namespace for: retention=%s, resolution=%s",
attrs.Retention.String(), attrs.Resolution.String())
}
default:
metricsType := common.attributes.MetricsType
err = fmt.Errorf("invalid write request metrics type: %s (%d)",
metricsType.String(), uint(metricsType))
}
if err != nil {
return err
}
namespaceID := namespace.NamespaceID()
session := namespace.Session()
return session.WriteTagged(namespaceID, id, common.tagIterator,
w.timestamp, w.value, common.unit, common.annotation)
}
type writeRequestCommon struct {
store *localStorage
annotation []byte
unit xtime.Unit
id string
tagIterator ident.TagIterator
attributes storage.Attributes
}
type writeRequest struct {
writeRequestCommon *writeRequestCommon
timestamp time.Time
value float64
}
func newWriteRequest(writeRequestCommon *writeRequestCommon, timestamp time.Time, value float64) execution.Request {
return &writeRequest{
writeRequestCommon: writeRequestCommon,
timestamp: timestamp,
value: value,
}
}
type multiFetchResult struct {
sync.Mutex
result *storage.FetchResult
err xerrors.MultiError
dedupeFirstAttrs storage.Attributes
dedupeMap map[string]multiFetchResultSeries
}
type multiFetchResultSeries struct {
idx int
attrs storage.Attributes
}
func (r *multiFetchResult) add(
attrs storage.Attributes,
result *storage.FetchResult,
err error,
) {
r.Lock()
defer r.Unlock()
if err != nil {
r.err = r.err.Add(err)
return
}
if r.result == nil {
r.result = result
r.dedupeFirstAttrs = attrs
return
}
r.result.HasNext = r.result.HasNext && result.HasNext
r.result.LocalOnly = r.result.LocalOnly && result.LocalOnly
// Need to dedupe
if r.dedupeMap == nil {
r.dedupeMap = make(map[string]multiFetchResultSeries, len(r.result.SeriesList))
for idx, s := range r.result.SeriesList {
r.dedupeMap[s.Name()] = multiFetchResultSeries{
idx: idx,
attrs: r.dedupeFirstAttrs,
}
}
}
for _, s := range result.SeriesList {
id := s.Name()
existing, exists := r.dedupeMap[id]
if exists && existing.attrs.Resolution <= attrs.Resolution {
// Already exists and resolution of result we are adding is not as precise
continue
}
// Does not exist already or more precise, add result
var idx int
if !exists {
idx = len(r.result.SeriesList)
r.result.SeriesList = append(r.result.SeriesList, s)
} else {
idx = existing.idx
r.result.SeriesList[idx] = s
}
r.dedupeMap[id] = multiFetchResultSeries{
idx: idx,
attrs: attrs,
}
}
}
type multiFetchTagsResult struct {
sync.Mutex
result *storage.SearchResults
err xerrors.MultiError
dedupeMap map[string]struct{}
}
func (r *multiFetchTagsResult) add(
result *storage.SearchResults,
err error,
) {
r.Lock()
defer r.Unlock()
if err != nil {
r.err = r.err.Add(err)
return
}
if r.result == nil {
r.result = result
return
}
// Need to dedupe
if r.dedupeMap == nil {
r.dedupeMap = make(map[string]struct{}, len(r.result.Metrics))
for _, s := range r.result.Metrics {
r.dedupeMap[s.ID] = struct{}{}
}
}
for _, s := range result.Metrics {
id := s.ID
_, exists := r.dedupeMap[id]
if exists {
// Already exists
continue
}
// Does not exist already, add result
r.result.Metrics = append(r.result.Metrics, s)
r.dedupeMap[id] = struct{}{}
}
}