안녕하세요. 데브시스터즈 데이터플랫폼그룹의 박기범, 서혜인입니다.
여러분은 사업 지표를 얼마나 신속하고 정확하게 확인하고 싶으신가요? 아마 대부분 가능한 빠른 시점에 정확한 지표를 확인하고 싶어 하실 겁니다. 하지만, 데이터팀 입장에서는 사용자가 원하는 수준으로 정확하고 신속한 지표 서비스를 제공하기 위해 기술적으로 해결해야 하는 많은 문제들에 직면하게 됩니다. 이번 글에서는 데브시스터즈가 지난 수개월간 준실시간 지표 서비스를 도입하는 과정에서 마주하게 된 여러 기술적 도전 과제들과 해결 과정에 대해 소개해드리고자 합니다.
게임 런칭 당일에 어떤 지표를 볼 수 있나요?
신규 게임이 출시되는 날, 모든 구성원의 관심은 게임 지표에 집중됩니다.
현재 게임을 접속한 유저 수가 어떻게 되나요?
지금까지 발생한 매출 중에 가장 구매가 많은 상품은 무엇인가요?
출시 직후 촌각을 다투는 상황에서는, 게임팀에서 유저의 접속 현황과 매출 성과에 따라 빠르게 의사결정을 내릴 수 있도록 “정확하고 신속하게” 지표를 확인할 수 있어야 합니다. 정확하지 않은 지표는 자칫하면 잘못된 의사결정을 초래할 수 있으며, 지표 조회가 지연되면 중요한 순간에 빠른 결정을 내리기 어려워지기 때문입니다.
이런 이유로 대부분의 데이터 조직은 짧은 지연 시간으로 주요 지표를 확인할 수 있는 준실시간(near real-time) 지표 서비스 도입을 고려하게 됩니다. 하지만, 왜 많은 회 사들이 준실시간 지표 서비스 도입에 어려움을 겪고 있을까요?
각 회사마다 마주한 상황은 다르겠지만, 데브시스터즈에서 그동안 준실시간 지표 서비스를 도입하지 못한 데는 다양한 이유들이 있었습니다.
기존 지표 환경
데브시스터즈에서는 2015년부터 Spark 과 Elastic Stack 기반의 빅데이터 플랫폼을 운영해 왔습니다.
2021년에는 Databricks를 도입하여 Lakehouse Architecture와 Medallion Architecture 기반의 데이터 환경을 구축했습니다. 이를 통해, DB와 로그 등 게임 서비스와 여러 플랫폼에서 발생하는 다양한 데이터를 가공 및 집계하여 지표 서비스에 활용 중이었습니다.
Databricks Data+AI World Tour 2023 Keynote: Databricks 도입 후의 Devsisters
해당 환경에서 지표를 서비스하는 방법은 크게 세 가지로 구성되어 있었습니다.
- 게임 서버에 연결된 세션 수를 기반으로 동시 접속자 (이하 '동접') 지표를 집계합니다.
- 시간별 배치로 운영계 DB 테이블을, 일별 배치로 로그를 ETL하여 Data Warehouse와 Data Mart에 적재합니다.
- Elasticsearch 에 적재된 로그 원장을 기반으로 Kibana 에서 준실시간으로 갱신과 조회가 가능한 대시보드를 구성하여 활용합니다.
기존 지표 환경의 한계
이처럼 기존 지표 환경에서도 게임 런칭 당일 제공할 수 있는 지표들이 존재하였지만, 사용자 요구를 완전히 충족시키기엔 한계가 있었습니다.
지금 국가별 동접은 볼 수는 없나요? iOS 동접은 어떤가요?
기존 동접 지표는 서버 입장에서의 세션 수만 수집했기 때문에 국가별, 스토어별 세부 동접 지표는 확인할 수 없었습니다. 특히 게임 런칭 초기에는 특정 국가의 반응, 스토어 및 OS별 접속 현황, 10분 단위의 접속자 추이를 빠르게 파악할 필요가 있었지만, 이를 확인할 수 있는 방법이 없어 불편했습니다. 반면 DB 기반의 시간별 지표에서는 여러 차원에서 지표를 조회할 수는 있었지만, 지표가 집계되기까지 한시간 가량을 기다려야 했기에 신속하게 의사결정을 내리기 어려웠습니다.
키바나 대시보드가 동작하지 않아요
Elastic Stack 기반으로 제공되는 키바나 준실시간 대시보드를 통해 여러 지표를 빠르게 확인할 수 있었지만, 게임 런칭과 같이 대규모 트래픽이 몰릴 때는 운영에 어려움이 있었습니다. 게임 CBT 시기에는 문제가 없었지만, 런칭 당일 초당 수십만 건의 로그를 처리할 수 있는 수준으로 Elastic Stack을 운영하기에는 높은 비용이 발생하게 되었습니다. 이로 인해, 게임 런칭 때는 키바나 기반의 대시보드를 제한적으로 활용할 수 밖에 없었습니다.
키바나 대시보드 지표가 시간별, 일별 지표와 달라요
제한적으로 Kibana 대시보드를 활용할 때에도 대시보드의 일부 지표가 실제와 다르게 집계되어 혼선이 발생하기도 했습니다. Elastic Stack에서는 로그 원장을 단순 집계하여 지표를 제공했기 때문에, 국가별 환율을 반영한 원화 환산 매출이나 게임 상품 이름과 같은 메타 정보를 대시보드에서 활용하기에 어려움이 있었습니다. Elasticsearch Ingest Pipelne 과 같은 기능을 활용해 이를 개선할 수는 있었으나 구현과 관리가 복잡했고, 기존 시간별/일별 지표는 SQL로 ETL되므로 동일한 로직을 새로 구현하는 것 역시 부담이 컸습니다.
새로운 준실시간 지표 파이프라인
위와 같은 한계로 인해 게임 런칭 시 데이터 기반 의사결정이 지연되는 문제를 여러 차례 경험하면서, 이를 해결하기 위한 새로운 준실시간 지표 파이프라인을 개발하게 되었습니다.
이 목표를 달성하기 위해 다음과 같은 요구사항을 설정했습니다.
-
지표의 범위: 기존의 일별/시간별 지표로 대체할 수 없는 즉 각 확인이 필요한 지표만 준실시간 지표 파이프라인에 포함하기로 했습니다. 그간의 사용자 요청 사항과 인터뷰를 기반으로, 국가, OS, 스토어 등 다양한 기준에서 유저 회원가입 수, 동접, 매출과 같이 핵심 지표를 확인할 수 있는 것을 목표로 설정했습니다.
-
클라이언트 로그를 적극적으로 활용: 설정된 지표를 수집하기 위해서는 서버에서 수집하지 않던 클라이언트의 다양한 맥락 데이터를 활용할 필요가 있었습니다. 기존 클라이언트 로그 SDK의 기능을 확장하여, 주기적으로 로그 수집 서버에 현재의 상태를 보내는 '하트비트 로그'를 추가 하였고, 클라이언트에서 보내주는 정보의 양을 늘렸습니다. 그리고 준실시간 지표 파이프라인에서 클라이언트 로그를 적극적으로 활용하기로 했습니다.
-
10분 이내의 지연 시간: 신속하게 지표를 확인해야 한다는 목적을 달성할 수 있도록, SLO(Service Level Objective) 를 10분으로 설정하였고, 게임 런칭과 같은 고부하 상황에서도 이 요구사항을 만족하는 것을 목표로 설정했습니다.
-
적절한 데이터 인프라 비용: 준실시간 지표는 주로 게임 출시 시 활용되지만, 정기 업데이트나 이슈 상황에서도 사용될 수 있어야 했습니다. 이를 위해, 파이프라인 운영 비용이 게임 인프라에 부담되지 않도록 설계 단계에서부터 기존 파이프라인의 비용을 참고하여 적정 비용 목표를 설정했습니다.
-
SQL 기반의 ETL: 매출을 비롯한 준실시간 지표는 정확하게 집계되어야 하는 동시에 여러 복잡한 비즈니스 로직이 반영되어야 했습니다. 예를 들어, 국가별 매출을 집계할 때는 통화 환율 정보를 반영해야 하며, 환불이나 쿠폰 사용 등 스토어별 결제 플로우에 맞는 복잡한 집계 공식이 반영되어야 했습니다. 이러한 비즈니스 로직을 지표 집계에 반영하기 위해서는 SWE 또는 데이터 엔지니어가 로직을 직접 개발하기보다는 DW 엔지니어 또는 데이터 분석가가 로직 개발에 참여할 수밖에 없었습니다. 또한, 이들의 개발 효율성을 높이기 위해서는 PySpark과 같은 범용적 프로그래밍 언어보다는 SQL로 집계 로직을 개발할 수 있어야 했습니다.
스펙
앞서 설명한 요구사항을 만족할 수 있도록 아래와 같이 구성된 스트리밍 기반의 ETL 파이프라인을 구성하여 준실시간 지표를 서비스하기로 했습니다.
- 스트리밍: Kafka로부터 로그를 실시간으로 읽을 수 있어야 합니다. 앞서 언급한 클라이언 트 로그를 비롯해 데브시스터즈에서 발생한 다양한 로그들은 모두 Kafka로 인입되고 있으며, 로그를 스트리밍 처리하여 가공할 수 있어야 합니다.
- 전처리: Kafka로부터 읽어온 로그를 집계에 활용할 수 있는 형태로 가공하는 작업이 필요합니다.
- 집계: 가공된 데이터를 지표에 활용할 수 있도록 다양한 차원의 조합으로 지표를 집계해야 합니다.
- 시각화: 집계가 완료된 데이터를 사용자들이 조회할 수 있도록 대시보드로 서비스해야 합니다.
아래는 준실시간 지표 서비스의 흐름을 각 단계별로 추상화한 그림입니다. 그럼, 각 단계별로 구현해야 하는 상세한 스펙에 대해서 알아보겠습니다.
1. 스트리밍
스트리밍 단계에서 구현해야 하는 주요 스펙은 하트비트 로그의 Kafka topic을 분리하는 것이었습니다. 일반적인 로그는 다음날 자정 일 단위 배치를 통해 S3에 저장됩니다. 하지만, 준실시간 지표 서비스에 사용되는 하트비트 로그는 데이터 양이 많아 S3로 저장하는 데 드는 비용이 매우 많이 들지만 분석에 활용되는 빈도는 낮았습니다. 따라서 하트비트 로그를 별도의 Kafka topic으로 분리하여 기존 파이프라인에서 적재되지 않도록 Log Collection 모듈의 로직을 수정해야 했습니다.
2. 전처리
전처리 단계에서는 매출 집계에 사용되는 로그를 추출하기 위한 작업이 필요했습니다. 매출 로그는, 서버와 클라이언트에서 발생하는 로그 각각에 집계에 필요한 정보가 저장 되어 있기 때문에 서로 다른 두 개의 Kafka topic에 담긴 로그들 간의 JOIN 연산을 진행해야 했습니다.
얼핏 보면 데이터 파이프라인에서 쉽게 구현이 가능해 보이는 스펙이지만 스트리밍 데이터로부터 이를 구현할 때는 몇 가지 어려움들이 숨어있었습니다. 스트리밍에서는 Kafka topic마다 하나의 stream을 생성하는데, 서로 다른 두 Kafka topic의 로그를 조인하려면 stream-stream JOIN 방식이 필요합니다. 본래 JOIN은 데이터가 확정된 (INSERT가 완료된) 테이블들의 레코드를 조합하여 하나의 컬럼으로 표현할 때 사용됩니다. 따라서 계속해서 데이터가 인입되는 스트리밍에서는 각각의 데이터가 확정되는 (INSERT가 완료되는) 시점을 지정하여 JOIN의 기준점으로 사용해야 합니다. 이를 위해, 특정 시점을 기준으로 늦게 들어오는 데이터는 연산에서 DROP 할 수 있는 watermark 시간을 정해주어야 합니다.
# log 에 찍힌 timestamp 를 기준으로 watermark 를 1분으로 지정한다.
silver_df = (
bronze_df.select(
to_timestamp(col("timestamp")).alias("log_datetime")
).withWatermark("log_datetime", "1 minutes")
)
...
또한 각각의 stream 의 데이터 인입 완료 시점이 다를 수 있기 때문에 JOIN 할 다른 stream 의 데이터를 대기하는 시간을 지정해주어야 합니다.
revenue_df = (
server_df.join(
client_df,
[
server_df.log_datetime < client_df.log_datetime + expr("INTERVAL 5 MINUTES"),
server_df.log_datetime + expr("INTERVAL 5 MINUTES") >= client_df.log_datetime,
server_df.id == client_df.id,
],
"leftOuter",
)
.select(
# column from client_df,
# column from server_df
)
...
)
...
이 설정값들은 최종적인 지연시간을 고려하여 적절하게 설정해야 합니다. 너무 큰 값으로 설정하면 데이터가 미처 들어오기 전에 JOIN이 이루어져 null 값으로 처리될 위험이 있고, 반대로 너무 작은 값으로 설정하면 지연시간이 커져 최종 지표 제공에 영향을 줄 수 있습니다. 따라서 적절한 균형을 찾아야 합니다.
또한, 이 과정에서 중요한 요소는 State (StateStore) 입니다. 파이프라인 내부에서는 JOIN이 이루어지기까지, 즉 watermark를 기준으로 데이터가 모두 들어왔다고 판단하기까지 데이터가 들어올 때마다 연산하여 중간값인 state를 저장하고 업데이트합니다. 데이터의 인입이 완료되면 state 업데이트도 완료되고 테이블에 레코드가 업데이트됩니다. 따라서 state 연산의 복잡도 그리고 state 저장 메모리 를 스트리밍 구현 단계에서 염두에 두어야 합니다.
3. 집계
집계 단계에서는, 위에서 언급한 복잡한 집계 로직을 효율적으로 운영하기 위한 요구사항들이 존재했습니다.
- SQL을 활용할 수 있는가
- 내부 환율 테이블과 메타 테이블에 접근하여 집계 과정에서 활용할 수 있는가
- 하트비트 로그와 같이, 양이 매우 많은 경우에도 집계 시간이 지연되지 않게 클러스터를 효율적으로 관리하는 것이 가능한가
4. 시각화
마지막 시각화 단계에서는, 준실시간 지표를 조회하기 위해 일반적인 BI 대시보드와 달리 아래 특징적인 요구사항들이 존재했습니다.
- 자동으로 대시보드 내의 데이터가 짧은 간격으로 갱신될 수 있는가
- 분 단위 또는 초 단위의 시계열 그래프를 표현하기에 적합한 대시보드인가
이처럼, 준실시간 파이프라인을 구축하기 위해서는 각 단계 별로 요구되는 스펙들이 존재하였고 이러한 요구사항들을 구현할 수 있는 플랫폼을 선택하기 위해 다양한 프레임워크들을 검토해 보았습니다.
적절한 플랫폼 선택하기
스트리밍 파이프라인
시도 1. KSQL
위 1. 스트리밍, 2. 전처리, 3. 집계 단계를 수행하는 스트리밍 파이프라인의 플랫폼으로 검토한 첫 번째 프레임워크는 KSQL 이었습니다.
KSQL은 Kafka streams를 기반으로 한 프레임워크로, 기존 데이터 플랫폼에서 운영 중인 Kafka 클러스터를 그대로 사용할 수 있다는 장점이 있었습니다. 또한, SQL과 유사한 문법으로 DW 엔지니어/분석가의 러닝커브가 낮다는 점도 큰 장점이었습니다.
하지만 KSQL을 운영데이터에 적용해 보니 데이터 처리 성능 이슈로 인해 정확성 측면에서 문제가 발생했습 니다. KSQL에서는 별도의 Kafka topic을 생성해 aggregation, join에서 발생하는 state를 저장합니다. 이때 Kafka message의 key는 group by에 포함된 column 들로 지정되며 사용자가 임의로 변경할 수 없고, 이를 기준으로 partition을 나누게 됩니다.
데브시스터즈의 게임들은 한국 사용자 비율이 매우 높기 때문에, 국가별 지표를 계산하는 과정에서 (group by country_code
) 파티션이 편향되는 문제가 발생했습니다. 개발 환경에서는 데이터의 양이 적어 문제가 되지 않았지만 운영데이터는 양이 많아 파티션의 편향이 데이터 처리 속도에 큰 영향을 주었습니다.
특정 파티션에서 많은 데이터가 처리되며 Lag이 커졌고, 결국 집계되는 시점에 이미 KSQL 에 들어온 데이터가 누락되어 지표에 큰 오차가 발생했습니다. 이로 인해, KSQL은 후보에서 제외되었습니다.
시도 2. Databricks의 Delta Live Table (이하 DLT)
다음으로 살펴본 것은, Databricks의 DLT 서비스였습니다. DLT 서비스는 Databricks에서 제공하는 스트리밍 프레임워크로, 스트리밍 및 배치 ETL 방식을 간편하게 처리할 수 있는 기능을 제공합니다.
기본적으로 DLT는 Spark Streaming을 기반으로 구현되며, Python뿐만 아니라 SQL 언어도 지원합니다.
이로 인해, SQL 개발자가 손쉽게 파이프라인에 집계 지표(Gold 영역)를 추가할 수 있다는 점이 큰 장점이었습니다. 또한, Web UI를 통해 테이블 간 구조를 시각적으로 쉽게 확인할 수 있고, 실행로그 및 실행시간 등을 한 곳에서 볼 수 있어 파이프라인 운영 측면에서 매우 유용했습니다. 스트리밍 파이프라인을 UI를 통해 간편하게 생성할 수 있고 Restart 버튼을 통해 재실행할 수 있어 유지보수 관점에서도 간편한 점이 많았습니다.
하지만 PoC 과정에서, DLT는 저희가 요구한 일부 조건을 충족하는 데 어려운 부분이 있었습니다.
지표 집계(Gold 영역)에서, 많은 양의 하트비트 로그와 복잡한 매출 집계 쿼리를 처리하는 과정에서 state 연산으로 인한 부하가 발생했고, 이로 인해 파이프라인의 전반적인 지연이 초래되었습니다. 이 문제로 인해 stream 데이터를 처리하는 Silver 영역과 지표를 집계하는 Gold 영역의 파이프라인을 별도로 분리해야 할 필요성이 제기되었습니다.
하지만, DLT 플랫폼에서는 각 파이프라인별에 개별 클러스터가 할당되며, 다른 작업에서 해당 클러스터를 공유해서 사용하는 것이 불가능했습니다. 파이프라인을 분리하게 되면, Silver 영역과 Gold 영역 각각의 파이프라인에서 발생하는 클러스터 비용의 총합이 파이프라인을 운영하는 데 배정된 목표 금액 치를 넘어서는 문제가 발생했습니다. 이에 따라 조금 더 저렴하게 운영할 수 있는 방법을 찾아 나서게 되었습니다.
시도 3. Spark Streaming
마지막으로 검토한 방식은 DLT 서비스를 사용하지 않고 Spark Streaming을 직접 운영하는 것이었습니다.
해당 방식에서는 DLT와 달리 managed 되어 제공되는 뷰가 없기 때문에 Spark 설정, 클러스터 관리, 그리고 모니터링 방법 등을 작업자가 모두 직접 구축해야 했습니다.
하지만 Spark Streaming을 직접 관리하면서 얻을 수 있는 장점도 있었습니다. 가장 큰 장점은, 단일 Spark 클러스터를 stream 데이터를 처리하는 Silver 영역과 지표를 집계하는 Gold 영역의 파이프라인에서 공유하는 것이 가능해졌고 해당 클러스터를 파이프라인 내의 별도의 Spark 작업에서도 사용하는 것이 가능해졌습니다. 이를 통해, 비용 목표 금액 치를 충족하는 동시에 지표 서비스 운영에 필요한 다양한 작업들을 효율적으로 운영하는 것이 가능해졌습니다.
이러한 장점들을 고려하여, 저희는 Spark Streaming을 직접 운영하는 방식을 스트리밍 파이프라인의 플랫폼으로 최종적으로 선택하게 되었습니다.
대시보드
대시보드의 경우, 기존 데이터 플랫폼에서도 활용 중인 Tableau, Quicksight 등 다양한 선택지가 있었지만, 위에서 언급한 4. 시각화 단계에 요구되는 두 가지 요건을 충족하면서 사내 구성원들에게 친숙한 환경인 Kibana를 선택하게 되었습니다.
1.자동으로 대시보드 내의 데이터가 짧은 간격으로 갱신될 수 있는가
2.분 단위 또는 초 단위의 시계열 그래프를 표현하기에 적합한 대시보드인가
Kibana 에서 제공하는 대시보드의 경우, Auto Refresh 기능을 통해 대시보드에서 조회하는 데이터를 원하는 간격으로 갱신할 수 있습니다. 또한, 그래프상의 timeseries 축을 자유롭게 조정할 수 있기에 위 두 가지 요건을 모두 충족할 수 있었습니다.
윗글에서는 ELK 스택으로 준실시간 지표 서비스를 구현하는 데 어려움이 있었다고 언급되어 있어, 신규 플랫폼에서 Kibana 대시보드의 역할에서 어떤 점이 달라지는 것인지 궁금한 분들도 계실 텐데요. 기존 데이터 플랫폼 구조에서는 Kibana에서 많은 양의 로그를 집계하는 연산을 사용하게 되면 클러스터에 큰 부하가 발생해 Elasticsearch 검색 기능과 같은 다른 서비스에도 문제를 유발하였습니다. 또한, 집계에 필요한 환율 테이블과 다른 메타 테이블에 대한 접근도 불가능하여 정확한 지표 집계에 어려움이 있었습니다.
하지만, Spark Streaming 기반의 신규 플랫폼에서는 집계 파이프라인에서 이미 집계가 완료된 데이터(Gold)를 Kibana 대시보드에서 조회하게 되면서 복잡한 연산 과정 없이 지표를 조회할 수 있게 되었습니다. 이로 인해 정확한 지표 집계와 부하 문제 해결이 동시에 가능해졌습니다. 이러한 개선된 구조는 다음 파트에서 자세히 다룰 예정입니다.
최종 구조
해당 파이프라인은 준실시간 지표 서비스에 요구되는 조건과 스펙들을 모두 충족하기 위해 구성된 최종 아키텍쳐입니다. 이 아키텍쳐는 데브시스터즈에서 운영 중인 게임 및 플랫폼에서 발생하는 로그 데이터가 준실시간 수준의 짧은 레이턴시 안에 사용자들에게 지표로 조회되기까지의 과정을 포함하고 있습니다. 그럼, 각 영역들의 역할과 구현 방식에 관해서 확인해 보겠습니다.
(Spark Streaming) Kafka topic에 저장된 원본 로그는 Spark Streaming에서 전처리 과정을 거치게 됩니다. topic으로부터 읽어오는 원본 로그는 위에서 언급했듯이 양은 많지만, 활용 빈도는 낮아 Bronze Layer 테이블로 적재하지 않았습니다.
(Silver Layer) 원본 로그들은 타입 지정과 같은 기본적인 전처리와 함께 앞서 언급된 서로 다른 stream 로그 간의 JOIN 작업이 이루어져 Delta Table 형태로 저장되게 됩니다.
(Gold Layer) Spark Streaming에서 Delta Lake 형태의 테이블로 소스데이터가 저장되었기 때문에 데이터 집계 과정은 일반적인 집계 파이프라인과 유사한 구조로 구성할 수 있었습니다. 이 과정에서 Spark Streaming의 큰 장점인 단일 클러스터 공유를 통해 집계 작업을 효율적으로 운영할 수 있었습니다. 배치 작업의 경우, 기존에도 팀 내에서 활발하게 사용하고 있는 Airflow를 사용하여 2분 간격으로 Micro batch 방식으로 실행되도록 설정했습니다.
앞서 살펴보았듯이, 지표 집계를 위해서는 복잡한 집계 로직 구현과 환율 테이블을 사용하는 등 SQL 개발자가 자유롭게 개발이 가능한 환경이 필요했는데요, 현재의 환경, Delta Lake 형태로 저장된 로그 데이터를 Databricks SQL로 집계하고 Airflow 배치를 통해 실행하는 환경은 SQL 개발자에게 매우 친숙한 환경이었습니다. (실제로도 기존 데이터 플랫폼에서 사용 중인 집계 쿼리를 재활용하는 것이 가능했습니다.) 이를 통해, 5% 이내의, 준실시간 지표로써는 상당히 낮은 오차율의 지표를 집계하는 것이 가능해졌고 새로운 지표가 추가되거나 로직 수정이 필요한 경우에도 SQL에 익숙한 DW 엔지니어 또는 데이터 분석가가 아주 쉽게 코드를 수정할 수 있는 환경이 갖춰지게 되었습니다.
(Send Metric Task) 마지막 작업은, 집계된 데이터를 Kibana 대시보드에서 조회하기 위해 Elasticsearch로 데이터를 전송하는 작업입니다. 해당 작업에서도 마찬가지로 Spark Streaming에서 사용한 클러스터를 사용하여 비용 효율화를 달성할 수 있었습니다. 각각의 Gold 지표 테이블에서 최근 업데이트된 레코드들을 Elasticsearch로 보내는 spark job을 병렬적으로 제출하여 신규 레코드가 최대한 빠르게 Kibana 에서 조회될 수 있도록 구현했습니다.
def send_to_es(table_name):
# 최근 20분 내에 업데이트된 레코드를 ES로 보내는 작업
df = spark.read(f"{table_name}").where("updated_at" >= current_timestamp() - expr("INTERVAL 20 MINUTE"))
(
df.write.format("org.elasticsearch.spark.sql")
.option("es.nodes.wan.only", True)
.option("es.net.ssl", True)
.option("es.nodes", "{ES_HOST}")
.option("es.net.http.auth.user", "{ES_ID}")
.option("es.net.http.auth.pass", "{ES_PASSWORD}")
.option("es.mapping.id", "document_id")
.mode("append")
.save("{ES_INDEX}")
)
# 쓰레드 풀을 이용한 병렬 작업
with ThreadPoolExecutor(max_workers=thread_pool_size) as executor:
# Spark 작업 제출
futures = {}
for table_name in gold_table_list:
futures[executor.submit(send_to_es, df, table_name)] = table_name
for future in as_completed(futures):
table_name = futures[future]
try:
future.result()
print(f"Finish: {table_name} send to ES")
except Exception as e:
print(f"Failed: {table_name} send to ES\n{e}")
이처럼, 집계가 완료된 메트릭을 Kibana 대시보드에서 조회하게 되면서 기존 데이터 플랫폼의 ELK 스택에서 발생했던 문제들을 모두 해결하고 Kibana 대시보드의 장점만을 활용할 수 있게 되었습니다.
준실시간 지표 서비스를 도입하고 일어난 일
오랜 준비 과정 끝에, 마침내 준실시간 지표 서비스는 지난 2024년 6월 쿠키런: 모험의 탑이 출시되는 날 구성원들에게 처음으로 공개되었습니다.
서비스에 대한 관심은 가히 폭발적이었으며, 런칭 당일 준실시간 대시보드에 접속한 사내 구성원의 비율은 무려 30%에 육박했습니다. (접속 권한이 없는 사용자들을 고려한다면, 대략 3명 중 2명의 구성원이 준실시간 대시보드에 접속한 셈입니다.)
준실시간 대시보드 지표는 게임팀의 마케팅 캠페인 전략 수립, 앱 푸시 송출 등 비즈니스 성과에 직접적으로 기여할 수 있는 의사결정들에 활용될 수 있었고, 다양한 분들이 지표 대시보드에 대해 긍정적 피드백을 보내주셨습니다.
쿠키런: 모험의 탑은 앱 출시 일주일 만에 200만 다운로드 달성 및 누적 매출 100억 원 돌파라는 놀라운 성과를 달성하게 되었는데, 준실시간 대시보드를 통해 이 성과를 만드는 데에 간접적으로 기여할 수 있게 되어 뿌듯했습니다.
데이터 임팩트를 함께 만들어갈 인재를 찾고 있습니다.
지표 서비스가 사용자들에게 이렇게까지 큰 관심을 받고 비즈니스 성과에 직접적으로 기여할 수 있는 경험은 매우 특별하고 드문 일이라 생각됩니다.
데브시스터즈 데이터플랫폼 그룹에서는 쿠키런: 킹덤, 쿠키런: 모험의 탑과 같이 운영 중인 게임은 물론, 런칭을 앞둔 다양한 게임 서비스에도 고도화된 데이터 플랫폼과 지표 서비스를 제공하기 위한 다양한 업무를 수행하고 있습니다.
데이터를 통해 비즈니스 성과를 만들어 나가는, 데이터 임팩트를 만드는 경험을 하고 싶으신 분들께서는 아래 채용 공고를 꼭 확인하셔서 저희의 여정에 함께할 수 있기를 기다리겠습니다.
감사합니다.
데브시스터즈는 최고의 인재를 찾고 있습니다.
데브시스터즈에서는 능력있는 [기술본부] Data Platform Software Engineer (경력)와 [기술본부] Data Warehouse Engineer (경력)를 찾고 있습니다.자세한 내용은 채용 사이트를 확인해주세요!